Skip to content

Commit

Permalink
Fix flaky test_dump_cluster_unresponsive_remote_worker (#5679)
Browse files Browse the repository at this point in the history
  • Loading branch information
crusaderky committed Jan 21, 2022
1 parent b341711 commit 337f152
Showing 1 changed file with 8 additions and 8 deletions.
16 changes: 8 additions & 8 deletions distributed/tests/test_utils_test.py
Original file line number Diff line number Diff line change
Expand Up @@ -10,6 +10,8 @@
import yaml
from tornado import gen

import dask.config

from distributed import Client, Nanny, Scheduler, Worker, config, default_client
from distributed.core import Server, rpc
from distributed.metrics import time
Expand Down Expand Up @@ -100,11 +102,8 @@ async def test_gen_cluster_parametrized_variadic_workers(c, s, *workers, foo):
)
async def test_gen_cluster_set_config_nanny(c, s, a, b):
def assert_config():
import dask

assert dask.config.get("distributed.comm.timeouts.connect") == "1s"
assert dask.config.get("new.config.value") == "foo"
return dask.config

await c.run(assert_config)
await c.run_on_scheduler(assert_config)
Expand Down Expand Up @@ -535,12 +534,11 @@ async def test_dump_cluster_state_unresponsive_local_worker(s, a, b, tmpdir):
@gen_cluster(
client=True,
Worker=Nanny,
config={"distributed.comm.timeouts.connect": "200ms"},
config={"distributed.comm.timeouts.connect": "600ms"},
)
async def test_dump_cluster_unresponsive_remote_worker(c, s, a, b, tmpdir):
addr1, addr2 = s.workers
clog_fut = asyncio.create_task(
c.run(lambda dask_scheduler: dask_scheduler.stop(), workers=[addr1])
c.run(lambda dask_scheduler: dask_scheduler.stop(), workers=[a.worker_address])
)
await asyncio.sleep(0.2)

Expand All @@ -549,7 +547,9 @@ async def test_dump_cluster_unresponsive_remote_worker(c, s, a, b, tmpdir):
out = yaml.safe_load(fh)

assert out.keys() == {"scheduler", "workers", "versions"}
assert isinstance(out["workers"][addr2], dict)
assert out["workers"][addr1].startswith("OSError('Timed out trying to connect to")
assert isinstance(out["workers"][b.worker_address], dict)
assert out["workers"][a.worker_address].startswith(
"OSError('Timed out trying to connect to"
)

clog_fut.cancel()

0 comments on commit 337f152

Please sign in to comment.