Skip to content

Commit

Permalink
Polish parsing of worker-saturation from config
Browse files Browse the repository at this point in the history
  • Loading branch information
crusaderky committed Nov 3, 2022
1 parent aa1c6d8 commit 3fec109
Show file tree
Hide file tree
Showing 2 changed files with 11 additions and 7 deletions.
17 changes: 11 additions & 6 deletions distributed/scheduler.py
Original file line number Diff line number Diff line change
Expand Up @@ -1646,13 +1646,18 @@ def __init__(
/ 2.0
)

sat = dask.config.get("distributed.scheduler.worker-saturation")
try:
self.WORKER_SATURATION = float(sat)
except ValueError:
raise ValueError(
f"Unsupported `distributed.scheduler.worker-saturation` value {sat!r}. Must be a float."
self.WORKER_SATURATION = dask.config.get(
"distributed.scheduler.worker-saturation"
)
if (
not isinstance(self.WORKER_SATURATION, (int, float))
or self.WORKER_SATURATION <= 0
):
raise ValueError( # pragma: nocover
"`distributed.scheduler.worker-saturation` must be a float > 0; got "
+ repr(self.WORKER_SATURATION)
)

self.transition_counter = 0
self._idle_transition_counter = 0
self.transition_counter_max = transition_counter_max
Expand Down
1 change: 0 additions & 1 deletion distributed/tests/test_scheduler.py
Original file line number Diff line number Diff line change
Expand Up @@ -505,7 +505,6 @@ def func(first, second):
"saturation_config, expected_task_counts",
[
(2.5, (5, 3)),
("2.5", (5, 3)),
(2.0, (4, 2)),
(1.1, (3, 2)),
(1.0, (2, 1)),
Expand Down

0 comments on commit 3fec109

Please sign in to comment.