2021-11-05 14:42:43 +00:00
|
|
|
'''
|
2021-11-24 17:13:02 +00:00
|
|
|
The hipster way to force SC onto the stdlib's "async": 'infection mode'.
|
2021-11-05 14:42:43 +00:00
|
|
|
|
|
|
|
'''
|
2022-04-12 18:24:30 +00:00
|
|
|
from typing import Optional, Iterable, Union
|
2020-12-10 18:49:11 +00:00
|
|
|
import asyncio
|
2021-11-07 22:05:40 +00:00
|
|
|
import builtins
|
2022-04-12 18:24:30 +00:00
|
|
|
import itertools
|
2021-11-07 22:05:40 +00:00
|
|
|
import importlib
|
2020-12-10 18:49:11 +00:00
|
|
|
|
2022-10-13 21:00:24 +00:00
|
|
|
from exceptiongroup import BaseExceptionGroup
|
2020-12-10 18:49:11 +00:00
|
|
|
import pytest
|
2021-11-07 22:05:40 +00:00
|
|
|
import trio
|
2020-12-10 18:49:11 +00:00
|
|
|
import tractor
|
2022-07-14 20:09:05 +00:00
|
|
|
from tractor import (
|
|
|
|
to_asyncio,
|
|
|
|
RemoteActorError,
|
|
|
|
)
|
2022-04-12 18:24:30 +00:00
|
|
|
from tractor.trionics import BroadcastReceiver
|
2020-12-10 18:49:11 +00:00
|
|
|
|
2021-11-05 14:42:43 +00:00
|
|
|
|
2022-07-14 20:09:05 +00:00
|
|
|
async def sleep_and_err(
|
|
|
|
sleep_for: float = 0.1,
|
|
|
|
|
|
|
|
# just signature placeholders for compat with
|
|
|
|
# ``to_asyncio.open_channel_from()``
|
|
|
|
to_trio: Optional[trio.MemorySendChannel] = None,
|
|
|
|
from_trio: Optional[asyncio.Queue] = None,
|
|
|
|
|
|
|
|
):
|
|
|
|
if to_trio:
|
|
|
|
to_trio.send_nowait('start')
|
|
|
|
|
2022-02-24 18:27:47 +00:00
|
|
|
await asyncio.sleep(sleep_for)
|
2020-12-10 18:49:11 +00:00
|
|
|
assert 0
|
|
|
|
|
|
|
|
|
2021-11-07 22:05:40 +00:00
|
|
|
async def sleep_forever():
|
|
|
|
await asyncio.sleep(float('inf'))
|
|
|
|
|
|
|
|
|
2021-11-18 19:01:13 +00:00
|
|
|
async def trio_cancels_single_aio_task():
|
|
|
|
|
|
|
|
# spawn an ``asyncio`` task to run a func and return result
|
|
|
|
with trio.move_on_after(.2):
|
|
|
|
await tractor.to_asyncio.run_task(sleep_forever)
|
|
|
|
|
|
|
|
|
|
|
|
def test_trio_cancels_aio_on_actor_side(arb_addr):
|
|
|
|
'''
|
|
|
|
Spawn an infected actor that is cancelled by the ``trio`` side
|
|
|
|
task using std cancel scope apis.
|
|
|
|
|
|
|
|
'''
|
|
|
|
async def main():
|
|
|
|
async with tractor.open_nursery(
|
|
|
|
arbiter_addr=arb_addr
|
|
|
|
) as n:
|
|
|
|
await n.run_in_actor(
|
|
|
|
trio_cancels_single_aio_task,
|
|
|
|
infect_asyncio=True,
|
|
|
|
)
|
|
|
|
|
|
|
|
trio.run(main)
|
|
|
|
|
|
|
|
|
2021-11-07 22:05:40 +00:00
|
|
|
async def asyncio_actor(
|
|
|
|
|
|
|
|
target: str,
|
|
|
|
expect_err: Optional[Exception] = None
|
|
|
|
|
|
|
|
) -> None:
|
|
|
|
|
2020-12-10 18:49:11 +00:00
|
|
|
assert tractor.current_actor().is_infected_aio()
|
2021-11-07 22:05:40 +00:00
|
|
|
target = globals()[target]
|
2020-12-10 18:49:11 +00:00
|
|
|
|
2021-11-07 22:05:40 +00:00
|
|
|
if '.' in expect_err:
|
|
|
|
modpath, _, name = expect_err.rpartition('.')
|
|
|
|
mod = importlib.import_module(modpath)
|
2021-11-17 18:32:42 +00:00
|
|
|
error_type = getattr(mod, name)
|
|
|
|
|
|
|
|
else: # toplevel builtin error type
|
|
|
|
error_type = builtins.__dict__.get(expect_err)
|
2021-11-07 22:05:40 +00:00
|
|
|
|
|
|
|
try:
|
|
|
|
# spawn an ``asyncio`` task to run a func and return result
|
|
|
|
await tractor.to_asyncio.run_task(target)
|
2021-11-17 18:32:42 +00:00
|
|
|
|
|
|
|
except BaseException as err:
|
2021-11-07 22:05:40 +00:00
|
|
|
if expect_err:
|
2021-11-17 18:32:42 +00:00
|
|
|
assert isinstance(err, error_type)
|
2021-11-07 22:05:40 +00:00
|
|
|
|
2021-11-24 17:13:02 +00:00
|
|
|
raise
|
2020-12-10 18:49:11 +00:00
|
|
|
|
|
|
|
|
2021-11-05 14:42:43 +00:00
|
|
|
def test_aio_simple_error(arb_addr):
|
2021-11-07 22:05:40 +00:00
|
|
|
'''
|
|
|
|
Verify a simple remote asyncio error propagates back through trio
|
|
|
|
to the parent actor.
|
|
|
|
|
2020-12-10 18:49:11 +00:00
|
|
|
|
2021-11-07 22:05:40 +00:00
|
|
|
'''
|
2020-12-10 18:49:11 +00:00
|
|
|
async def main():
|
2021-11-07 22:05:40 +00:00
|
|
|
async with tractor.open_nursery(
|
|
|
|
arbiter_addr=arb_addr
|
|
|
|
) as n:
|
|
|
|
await n.run_in_actor(
|
|
|
|
asyncio_actor,
|
|
|
|
target='sleep_and_err',
|
|
|
|
expect_err='AssertionError',
|
|
|
|
infect_asyncio=True,
|
|
|
|
)
|
2020-12-10 18:49:11 +00:00
|
|
|
|
2021-11-07 22:05:40 +00:00
|
|
|
with pytest.raises(RemoteActorError) as excinfo:
|
|
|
|
trio.run(main)
|
2021-11-05 14:42:43 +00:00
|
|
|
|
2021-11-07 22:05:40 +00:00
|
|
|
err = excinfo.value
|
|
|
|
assert isinstance(err, RemoteActorError)
|
|
|
|
assert err.type == AssertionError
|
2021-11-05 14:42:43 +00:00
|
|
|
|
2021-11-07 22:05:40 +00:00
|
|
|
|
|
|
|
def test_tractor_cancels_aio(arb_addr):
|
|
|
|
'''
|
|
|
|
Verify we can cancel a spawned asyncio task gracefully.
|
|
|
|
|
|
|
|
'''
|
|
|
|
async def main():
|
|
|
|
async with tractor.open_nursery() as n:
|
|
|
|
portal = await n.run_in_actor(
|
|
|
|
asyncio_actor,
|
|
|
|
target='sleep_forever',
|
2021-11-17 18:32:42 +00:00
|
|
|
expect_err='trio.Cancelled',
|
2021-11-07 22:05:40 +00:00
|
|
|
infect_asyncio=True,
|
|
|
|
)
|
2021-11-17 18:32:42 +00:00
|
|
|
# cancel the entire remote runtime
|
2021-11-07 22:05:40 +00:00
|
|
|
await portal.cancel_actor()
|
|
|
|
|
|
|
|
trio.run(main)
|
2021-11-05 14:42:43 +00:00
|
|
|
|
|
|
|
|
2021-11-18 14:35:59 +00:00
|
|
|
def test_trio_cancels_aio(arb_addr):
|
|
|
|
'''
|
|
|
|
Much like the above test with ``tractor.Portal.cancel_actor()``
|
|
|
|
except we just use a standard ``trio`` cancellation api.
|
|
|
|
|
|
|
|
'''
|
|
|
|
async def main():
|
|
|
|
|
|
|
|
with trio.move_on_after(1):
|
|
|
|
# cancel the nursery shortly after boot
|
|
|
|
|
|
|
|
async with tractor.open_nursery() as n:
|
2021-11-24 17:13:02 +00:00
|
|
|
await n.run_in_actor(
|
2021-11-18 14:35:59 +00:00
|
|
|
asyncio_actor,
|
|
|
|
target='sleep_forever',
|
|
|
|
expect_err='trio.Cancelled',
|
|
|
|
infect_asyncio=True,
|
|
|
|
)
|
|
|
|
|
|
|
|
trio.run(main)
|
|
|
|
|
|
|
|
|
2022-07-14 20:09:05 +00:00
|
|
|
@tractor.context
|
|
|
|
async def trio_ctx(
|
|
|
|
ctx: tractor.Context,
|
|
|
|
):
|
|
|
|
|
|
|
|
await ctx.started('start')
|
|
|
|
|
|
|
|
# this will block until the ``asyncio`` task sends a "first"
|
|
|
|
# message.
|
2022-07-15 00:44:10 +00:00
|
|
|
with trio.fail_after(2):
|
2022-07-14 20:09:05 +00:00
|
|
|
async with (
|
2022-10-11 17:48:27 +00:00
|
|
|
trio.open_nursery() as n,
|
|
|
|
|
2022-07-14 20:09:05 +00:00
|
|
|
tractor.to_asyncio.open_channel_from(
|
|
|
|
sleep_and_err,
|
|
|
|
) as (first, chan),
|
|
|
|
):
|
|
|
|
|
|
|
|
assert first == 'start'
|
|
|
|
|
|
|
|
# spawn another asyncio task for the cuck of it.
|
|
|
|
n.start_soon(
|
|
|
|
tractor.to_asyncio.run_task,
|
|
|
|
sleep_forever,
|
|
|
|
)
|
2022-07-14 20:35:41 +00:00
|
|
|
await trio.sleep_forever()
|
2022-07-14 20:09:05 +00:00
|
|
|
|
|
|
|
|
|
|
|
@pytest.mark.parametrize(
|
|
|
|
'parent_cancels', [False, True],
|
|
|
|
ids='parent_actor_cancels_child={}'.format
|
|
|
|
)
|
|
|
|
def test_context_spawns_aio_task_that_errors(
|
|
|
|
arb_addr,
|
|
|
|
parent_cancels: bool,
|
|
|
|
):
|
|
|
|
'''
|
|
|
|
Verify that spawning a task via an intertask channel ctx mngr that
|
|
|
|
errors correctly propagates the error back from the `asyncio`-side
|
2022-07-15 15:18:48 +00:00
|
|
|
task.
|
2022-07-14 20:09:05 +00:00
|
|
|
|
|
|
|
'''
|
|
|
|
async def main():
|
|
|
|
|
2022-10-11 17:35:55 +00:00
|
|
|
with trio.fail_after(2):
|
|
|
|
async with tractor.open_nursery() as n:
|
|
|
|
p = await n.start_actor(
|
|
|
|
'aio_daemon',
|
|
|
|
enable_modules=[__name__],
|
|
|
|
infect_asyncio=True,
|
|
|
|
# debug_mode=True,
|
|
|
|
loglevel='cancel',
|
|
|
|
)
|
|
|
|
async with p.open_context(
|
|
|
|
trio_ctx,
|
|
|
|
) as (ctx, first):
|
2022-07-14 20:09:05 +00:00
|
|
|
|
2022-10-11 17:35:55 +00:00
|
|
|
assert first == 'start'
|
2022-07-14 20:09:05 +00:00
|
|
|
|
2022-10-11 17:35:55 +00:00
|
|
|
if parent_cancels:
|
|
|
|
await p.cancel_actor()
|
2022-07-14 20:09:05 +00:00
|
|
|
|
2022-10-11 17:35:55 +00:00
|
|
|
await trio.sleep_forever()
|
2022-07-14 20:09:05 +00:00
|
|
|
|
|
|
|
with pytest.raises(RemoteActorError) as excinfo:
|
|
|
|
trio.run(main)
|
|
|
|
|
|
|
|
err = excinfo.value
|
|
|
|
assert isinstance(err, RemoteActorError)
|
|
|
|
if parent_cancels:
|
|
|
|
assert err.type == trio.Cancelled
|
|
|
|
else:
|
|
|
|
assert err.type == AssertionError
|
|
|
|
|
|
|
|
|
2021-11-17 18:32:42 +00:00
|
|
|
async def aio_cancel():
|
2021-11-24 17:13:02 +00:00
|
|
|
''''
|
|
|
|
Cancel urself boi.
|
2021-11-17 18:32:42 +00:00
|
|
|
|
|
|
|
'''
|
|
|
|
await asyncio.sleep(0.5)
|
|
|
|
task = asyncio.current_task()
|
|
|
|
|
|
|
|
# cancel and enter sleep
|
|
|
|
task.cancel()
|
|
|
|
await sleep_forever()
|
|
|
|
|
|
|
|
|
2021-11-05 14:42:43 +00:00
|
|
|
def test_aio_cancelled_from_aio_causes_trio_cancelled(arb_addr):
|
2021-11-17 18:32:42 +00:00
|
|
|
|
|
|
|
async def main():
|
|
|
|
async with tractor.open_nursery() as n:
|
2021-11-25 22:10:22 +00:00
|
|
|
await n.run_in_actor(
|
2021-11-17 18:32:42 +00:00
|
|
|
asyncio_actor,
|
|
|
|
target='aio_cancel',
|
2021-11-24 17:13:02 +00:00
|
|
|
expect_err='tractor.to_asyncio.AsyncioCancelled',
|
2021-11-17 18:32:42 +00:00
|
|
|
infect_asyncio=True,
|
|
|
|
)
|
|
|
|
|
|
|
|
with pytest.raises(RemoteActorError) as excinfo:
|
|
|
|
trio.run(main)
|
2021-11-05 14:42:43 +00:00
|
|
|
|
2021-11-24 17:13:02 +00:00
|
|
|
# ensure boxed error is correct
|
|
|
|
assert excinfo.value.type == to_asyncio.AsyncioCancelled
|
|
|
|
|
2021-11-05 14:42:43 +00:00
|
|
|
|
2021-11-22 18:27:16 +00:00
|
|
|
# TODO: verify open_channel_from will fail on this..
|
2021-11-20 17:47:03 +00:00
|
|
|
async def no_to_trio_in_args():
|
|
|
|
pass
|
|
|
|
|
|
|
|
|
|
|
|
async def push_from_aio_task(
|
|
|
|
|
|
|
|
sequence: Iterable,
|
|
|
|
to_trio: trio.abc.SendChannel,
|
2021-11-22 18:27:16 +00:00
|
|
|
expect_cancel: False,
|
|
|
|
fail_early: bool,
|
2021-11-20 17:47:03 +00:00
|
|
|
|
|
|
|
) -> None:
|
|
|
|
|
2021-11-22 18:27:16 +00:00
|
|
|
try:
|
|
|
|
# sync caller ctx manager
|
|
|
|
to_trio.send_nowait(True)
|
|
|
|
|
|
|
|
for i in sequence:
|
|
|
|
print(f'asyncio sending {i}')
|
|
|
|
to_trio.send_nowait(i)
|
|
|
|
await asyncio.sleep(0.001)
|
|
|
|
|
|
|
|
if i == 50 and fail_early:
|
|
|
|
raise Exception
|
|
|
|
|
2021-11-24 17:13:02 +00:00
|
|
|
print('asyncio streamer complete!')
|
2021-11-22 18:27:16 +00:00
|
|
|
|
|
|
|
except asyncio.CancelledError:
|
|
|
|
if not expect_cancel:
|
|
|
|
pytest.fail("aio task was cancelled unexpectedly")
|
|
|
|
raise
|
|
|
|
else:
|
|
|
|
if expect_cancel:
|
|
|
|
pytest.fail("aio task wasn't cancelled as expected!?")
|
|
|
|
|
2021-11-20 17:47:03 +00:00
|
|
|
|
2021-11-22 18:27:16 +00:00
|
|
|
async def stream_from_aio(
|
2021-11-20 17:47:03 +00:00
|
|
|
|
2021-11-22 18:27:16 +00:00
|
|
|
exit_early: bool = False,
|
|
|
|
raise_err: bool = False,
|
|
|
|
aio_raise_err: bool = False,
|
2022-04-12 18:24:30 +00:00
|
|
|
fan_out: bool = False,
|
2021-11-22 18:27:16 +00:00
|
|
|
|
|
|
|
) -> None:
|
2021-11-20 17:47:03 +00:00
|
|
|
seq = range(100)
|
|
|
|
expect = list(seq)
|
|
|
|
|
2021-11-22 18:27:16 +00:00
|
|
|
try:
|
|
|
|
pulled = []
|
|
|
|
|
|
|
|
async with to_asyncio.open_channel_from(
|
|
|
|
push_from_aio_task,
|
|
|
|
sequence=seq,
|
|
|
|
expect_cancel=raise_err or exit_early,
|
|
|
|
fail_early=aio_raise_err,
|
|
|
|
) as (first, chan):
|
|
|
|
|
|
|
|
assert first is True
|
|
|
|
|
2022-04-12 18:24:30 +00:00
|
|
|
async def consume(
|
|
|
|
chan: Union[
|
|
|
|
to_asyncio.LinkedTaskChannel,
|
|
|
|
BroadcastReceiver,
|
|
|
|
],
|
|
|
|
):
|
|
|
|
async for value in chan:
|
|
|
|
print(f'trio received {value}')
|
|
|
|
pulled.append(value)
|
|
|
|
|
|
|
|
if value == 50:
|
|
|
|
if raise_err:
|
|
|
|
raise Exception
|
|
|
|
elif exit_early:
|
|
|
|
break
|
|
|
|
|
|
|
|
if fan_out:
|
|
|
|
# start second task that get's the same stream value set.
|
|
|
|
async with (
|
2022-04-12 19:03:44 +00:00
|
|
|
|
|
|
|
# NOTE: this has to come first to avoid
|
|
|
|
# the channel being closed before the nursery
|
|
|
|
# tasks are joined..
|
2022-04-12 18:24:30 +00:00
|
|
|
chan.subscribe() as br,
|
2022-04-12 19:03:44 +00:00
|
|
|
|
|
|
|
trio.open_nursery() as n,
|
2022-04-12 18:24:30 +00:00
|
|
|
):
|
|
|
|
n.start_soon(consume, br)
|
|
|
|
await consume(chan)
|
|
|
|
|
|
|
|
else:
|
|
|
|
await consume(chan)
|
2021-11-22 18:27:16 +00:00
|
|
|
finally:
|
2021-11-20 17:47:03 +00:00
|
|
|
|
2021-11-22 18:27:16 +00:00
|
|
|
if (
|
|
|
|
not raise_err and
|
|
|
|
not exit_early and
|
|
|
|
not aio_raise_err
|
|
|
|
):
|
2022-04-12 18:24:30 +00:00
|
|
|
if fan_out:
|
|
|
|
# we get double the pulled values in the
|
|
|
|
# ``.subscribe()`` fan out case.
|
|
|
|
doubled = list(itertools.chain(*zip(expect, expect)))
|
2022-04-12 20:13:33 +00:00
|
|
|
expect = doubled[:len(pulled)]
|
2022-04-12 20:13:33 +00:00
|
|
|
assert list(sorted(pulled)) == expect
|
2022-04-12 18:24:30 +00:00
|
|
|
|
|
|
|
else:
|
|
|
|
assert pulled == expect
|
2021-11-22 18:27:16 +00:00
|
|
|
else:
|
2022-04-12 20:13:33 +00:00
|
|
|
assert not fan_out
|
2021-11-22 18:27:16 +00:00
|
|
|
assert pulled == expect[:51]
|
2021-11-20 17:47:03 +00:00
|
|
|
|
2021-11-22 18:27:16 +00:00
|
|
|
print('trio guest mode task completed!')
|
2021-11-05 14:42:43 +00:00
|
|
|
|
|
|
|
|
2022-04-12 18:24:30 +00:00
|
|
|
@pytest.mark.parametrize(
|
|
|
|
'fan_out', [False, True],
|
|
|
|
ids='fan_out_w_chan_subscribe={}'.format
|
|
|
|
)
|
|
|
|
def test_basic_interloop_channel_stream(arb_addr, fan_out):
|
2021-11-20 17:47:03 +00:00
|
|
|
async def main():
|
|
|
|
async with tractor.open_nursery() as n:
|
|
|
|
portal = await n.run_in_actor(
|
|
|
|
stream_from_aio,
|
|
|
|
infect_asyncio=True,
|
2022-04-12 18:24:30 +00:00
|
|
|
fan_out=fan_out,
|
2021-11-20 17:47:03 +00:00
|
|
|
)
|
|
|
|
await portal.result()
|
|
|
|
|
|
|
|
trio.run(main)
|
|
|
|
|
|
|
|
|
2021-11-22 18:27:16 +00:00
|
|
|
# TODO: parametrize the above test and avoid the duplication here?
|
|
|
|
def test_trio_error_cancels_intertask_chan(arb_addr):
|
|
|
|
async def main():
|
|
|
|
async with tractor.open_nursery() as n:
|
|
|
|
portal = await n.run_in_actor(
|
|
|
|
stream_from_aio,
|
|
|
|
raise_err=True,
|
|
|
|
infect_asyncio=True,
|
|
|
|
)
|
|
|
|
# should trigger remote actor error
|
|
|
|
await portal.result()
|
2021-11-20 17:47:03 +00:00
|
|
|
|
2022-10-13 21:00:24 +00:00
|
|
|
with pytest.raises(BaseExceptionGroup) as excinfo:
|
2021-11-22 18:27:16 +00:00
|
|
|
trio.run(main)
|
2021-11-05 14:42:43 +00:00
|
|
|
|
2022-10-13 21:00:24 +00:00
|
|
|
# ensure boxed errors
|
|
|
|
for exc in excinfo.value.exceptions:
|
|
|
|
assert exc.type == Exception
|
2021-11-05 14:42:43 +00:00
|
|
|
|
2021-11-22 18:27:16 +00:00
|
|
|
|
|
|
|
def test_trio_closes_early_and_channel_exits(arb_addr):
|
|
|
|
async def main():
|
|
|
|
async with tractor.open_nursery() as n:
|
|
|
|
portal = await n.run_in_actor(
|
|
|
|
stream_from_aio,
|
|
|
|
exit_early=True,
|
|
|
|
infect_asyncio=True,
|
|
|
|
)
|
|
|
|
# should trigger remote actor error
|
|
|
|
await portal.result()
|
|
|
|
|
|
|
|
# should be a quiet exit on a simple channel exit
|
|
|
|
trio.run(main)
|
|
|
|
|
|
|
|
|
|
|
|
def test_aio_errors_and_channel_propagates_and_closes(arb_addr):
|
|
|
|
async def main():
|
|
|
|
async with tractor.open_nursery() as n:
|
|
|
|
portal = await n.run_in_actor(
|
|
|
|
stream_from_aio,
|
|
|
|
aio_raise_err=True,
|
|
|
|
infect_asyncio=True,
|
|
|
|
)
|
|
|
|
# should trigger remote actor error
|
|
|
|
await portal.result()
|
|
|
|
|
2022-10-13 21:00:24 +00:00
|
|
|
with pytest.raises(BaseExceptionGroup) as excinfo:
|
2021-11-22 18:27:16 +00:00
|
|
|
trio.run(main)
|
|
|
|
|
2022-10-13 21:00:24 +00:00
|
|
|
# ensure boxed errors
|
|
|
|
for exc in excinfo.value.exceptions:
|
|
|
|
assert exc.type == Exception
|
2021-11-05 14:42:43 +00:00
|
|
|
|
|
|
|
|
2021-11-25 22:10:22 +00:00
|
|
|
@tractor.context
|
|
|
|
async def trio_to_aio_echo_server(
|
|
|
|
ctx: tractor.Context,
|
|
|
|
):
|
|
|
|
|
|
|
|
async def aio_echo_server(
|
|
|
|
to_trio: trio.MemorySendChannel,
|
|
|
|
from_trio: asyncio.Queue,
|
|
|
|
) -> None:
|
|
|
|
|
|
|
|
to_trio.send_nowait('start')
|
|
|
|
|
|
|
|
while True:
|
|
|
|
msg = await from_trio.get()
|
|
|
|
|
|
|
|
# echo the msg back
|
|
|
|
to_trio.send_nowait(msg)
|
|
|
|
|
|
|
|
# if we get the terminate sentinel
|
|
|
|
# break the echo loop
|
|
|
|
if msg is None:
|
|
|
|
print('breaking aio echo loop')
|
|
|
|
break
|
|
|
|
|
2022-07-14 20:09:05 +00:00
|
|
|
print('exiting asyncio task')
|
|
|
|
|
2021-11-25 22:10:22 +00:00
|
|
|
async with to_asyncio.open_channel_from(
|
|
|
|
aio_echo_server,
|
|
|
|
) as (first, chan):
|
|
|
|
|
|
|
|
assert first == 'start'
|
|
|
|
await ctx.started(first)
|
|
|
|
|
|
|
|
async with ctx.open_stream() as stream:
|
|
|
|
|
|
|
|
async for msg in stream:
|
|
|
|
print(f'asyncio echoing {msg}')
|
|
|
|
await chan.send(msg)
|
|
|
|
|
|
|
|
out = await chan.receive()
|
|
|
|
# echo back to parent actor-task
|
|
|
|
await stream.send(out)
|
|
|
|
|
|
|
|
if out is None:
|
|
|
|
try:
|
|
|
|
out = await chan.receive()
|
|
|
|
except trio.EndOfChannel:
|
|
|
|
break
|
|
|
|
else:
|
|
|
|
raise RuntimeError('aio channel never stopped?')
|
|
|
|
|
|
|
|
|
2021-11-28 02:55:04 +00:00
|
|
|
@pytest.mark.parametrize(
|
|
|
|
'raise_error_mid_stream',
|
|
|
|
[False, Exception, KeyboardInterrupt],
|
|
|
|
ids='raise_error={}'.format,
|
|
|
|
)
|
|
|
|
def test_echoserver_detailed_mechanics(
|
|
|
|
arb_addr,
|
|
|
|
raise_error_mid_stream,
|
|
|
|
):
|
2021-11-25 22:10:22 +00:00
|
|
|
|
|
|
|
async def main():
|
|
|
|
async with tractor.open_nursery() as n:
|
|
|
|
p = await n.start_actor(
|
|
|
|
'aio_server',
|
|
|
|
enable_modules=[__name__],
|
|
|
|
infect_asyncio=True,
|
|
|
|
)
|
|
|
|
async with p.open_context(
|
|
|
|
trio_to_aio_echo_server,
|
|
|
|
) as (ctx, first):
|
|
|
|
|
|
|
|
assert first == 'start'
|
|
|
|
|
|
|
|
async with ctx.open_stream() as stream:
|
|
|
|
for i in range(100):
|
|
|
|
await stream.send(i)
|
|
|
|
out = await stream.receive()
|
|
|
|
assert i == out
|
|
|
|
|
2021-11-28 02:55:04 +00:00
|
|
|
if raise_error_mid_stream and i == 50:
|
|
|
|
raise raise_error_mid_stream
|
|
|
|
|
2021-11-25 22:10:22 +00:00
|
|
|
# send terminate msg
|
|
|
|
await stream.send(None)
|
|
|
|
out = await stream.receive()
|
|
|
|
assert out is None
|
|
|
|
|
|
|
|
if out is None:
|
|
|
|
# ensure the stream is stopped
|
|
|
|
# with trio.fail_after(0.1):
|
|
|
|
try:
|
|
|
|
await stream.receive()
|
|
|
|
except trio.EndOfChannel:
|
|
|
|
pass
|
|
|
|
else:
|
|
|
|
pytest.fail(
|
|
|
|
"stream wasn't stopped after sentinel?!")
|
|
|
|
|
|
|
|
# TODO: the case where this blocks and
|
|
|
|
# is cancelled by kbi or out of task cancellation
|
|
|
|
await p.cancel_actor()
|
|
|
|
|
2021-11-28 02:55:04 +00:00
|
|
|
if raise_error_mid_stream:
|
|
|
|
with pytest.raises(raise_error_mid_stream):
|
|
|
|
trio.run(main)
|
|
|
|
|
|
|
|
else:
|
|
|
|
trio.run(main)
|