Skip to content
New issue

Have a question about this project? Sign up for a free GitHub account to open an issue and contact its maintainers and the community.

By clicking “Sign up for GitHub”, you agree to our terms of service and privacy statement. We’ll occasionally send you account related emails.

Already on GitHub? Sign in to your account

Remove update_graph_hlg #7502

Closed
wants to merge 4 commits into from
Closed
Show file tree
Hide file tree
Changes from 2 commits
Commits
File filter

Filter by extension

Filter by extension

Conversations
Failed to load comments.
Loading
Jump to
Jump to file
Failed to load files.
Loading
Diff view
Diff view
10 changes: 1 addition & 9 deletions distributed/client.py
Original file line number Diff line number Diff line change
Expand Up @@ -468,14 +468,6 @@ def __setstate__(self, state):
except ValueError:
c = get_client(address)
self.__init__(key, c)
c._send_to_scheduler(
{
"op": "update-graph",
"tasks": {},
"keys": [stringify(self.key)],
"client": c.id,
}
)
Comment on lines -471 to -478
Copy link
Member Author

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

From what I can tell this is not serving any purpose. The future itself lets the scheduler know that it is interested with a client_desires_keys so this is redundant and the only non-test invocation of the old API


def __del__(self):
try:
Expand Down Expand Up @@ -3036,7 +3028,7 @@ def _graph_to_futures(

self._send_to_scheduler(
{
"op": "update-graph-hlg",
"op": "update-graph",
"hlg": dsk,
"keys": list(map(stringify, keys)),
"priority": priority,
Expand Down
72 changes: 20 additions & 52 deletions distributed/diagnostics/tests/test_widgets.py
Original file line number Diff line number Diff line change
Expand Up @@ -7,11 +7,9 @@

import pytest
from packaging.version import parse as parse_version
from tlz import valmap

from distributed.client import wait
from distributed.utils_test import dec, gen_cluster, gen_tls_cluster, inc, throws
from distributed.worker import dumps_task

ipywidgets = pytest.importorskip("ipywidgets")

Expand Down Expand Up @@ -142,35 +140,20 @@ async def test_multi_progressbar_widget(c, s, a, b):


@mock_widget()
@gen_cluster()
async def test_multi_progressbar_widget_after_close(s, a, b):
Copy link
Member Author

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

This test is a bit odd and I couldn't make sense of the suffix _after_close. One thing that is unique about this test is that it tests the multiprogress bars explicitly with intermediate keys so I adjusted accordingly

Copy link
Member Author

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

Keeps failing. After some thought I'm not even sure if this is possible. I'll remove the test

s.update_graph(
tasks=valmap(
dumps_task,
{
"x-1": (inc, 1),
"x-2": (inc, "x-1"),
"x-3": (inc, "x-2"),
"y-1": (dec, "x-3"),
"y-2": (dec, "y-1"),
"e": (throws, "y-2"),
"other": (inc, 123),
},
),
keys=["e"],
dependencies={
"x-2": {"x-1"},
"x-3": {"x-2"},
"y-1": {"x-3"},
"y-2": {"y-1"},
"e": {"y-2"},
},
)
@gen_cluster(client=True)
async def test_multi_progressbar_widget_intermediates(c, s, a, b):
x1 = c.submit(inc, 1, key="x-1")
x2 = c.submit(inc, x1, key="x-2")
x3 = c.submit(inc, x2, key="x-3")
y1 = c.submit(dec, x3)
y2 = c.submit(dec, y1)
del x1, x2, x3, y1

p = MultiProgressWidget(["x-1", "x-2", "x-3"], scheduler=s.address)
await p.listen()

assert "x" in p.bars
await y2


@mock_widget()
Expand Down Expand Up @@ -232,32 +215,17 @@ def test_progressbar_cancel(client):


@mock_widget()
@gen_cluster()
async def test_multibar_complete(s, a, b):
s.update_graph(
tasks=valmap(
dumps_task,
{
"x-1": (inc, 1),
"x-2": (inc, "x-1"),
"x-3": (inc, "x-2"),
"y-1": (dec, "x-3"),
"y-2": (dec, "y-1"),
"e": (throws, "y-2"),
"other": (inc, 123),
},
),
keys=["e"],
dependencies={
"x-2": {"x-1"},
"x-3": {"x-2"},
"y-1": {"x-3"},
"y-2": {"y-1"},
"e": {"y-2"},
},
)

p = MultiProgressWidget(["e"], scheduler=s.address, complete=True)
@gen_cluster(client=True)
async def test_multibar_complete(c, s, a, b):
x1 = c.submit(inc, 1, key="x-1")
x2 = c.submit(inc, x1, key="x-2")
x3 = c.submit(inc, x2, key="x-3")
y1 = c.submit(dec, x3, key="y-1")
y2 = c.submit(dec, y1, key="y-2")
e = c.submit(throws, y2, key="e")
other = c.submit(inc, 123, key="other")

p = MultiProgressWidget([e.key], scheduler=s.address, complete=True)
await p.listen()

assert p._last_response["all"] == {"x": 3, "y": 2, "e": 1}
Expand Down
Loading