Skip to content

Commit c590e9a

Browse files
committed
[Data] Add descriptive error when using local:// paths with a zero-resource head node
Signed-off-by: Haichuan Hu <kaisennhu@gmail.com>
1 parent edc14c5 commit c590e9a

File tree

4 files changed

+137
-0
lines changed

4 files changed

+137
-0
lines changed

python/ray/data/_internal/util.py

Lines changed: 88 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -368,6 +368,94 @@ def _is_local_scheme(paths: Union[str, List[str]]) -> bool:
368368
return num == len(paths)
369369

370370

371+
def _validate_head_node_resources_for_local_scheduling(
372+
ray_remote_args: Dict[str, Any],
373+
*,
374+
op_name: str,
375+
required_num_cpus: int = 1,
376+
required_num_gpus: int = 0,
377+
required_memory: int = 0,
378+
) -> None:
379+
"""Ensure the head node has enough resources before pinning work there.
380+
381+
Local paths (``local://``) and other driver-local I/O force tasks onto the
382+
head node via ``NodeAffinitySchedulingStrategy``. If the head node was
383+
intentionally started with zero logical resources (a common practice to
384+
avoid OOMs), those tasks become unschedulable. Detect this upfront and
385+
raise a clear error with remediation steps.
386+
"""
387+
388+
# Ray defaults to reserving 1 CPU per task when num_cpus isn't provided.
389+
num_cpus = ray_remote_args.get("num_cpus", required_num_cpus)
390+
num_gpus = ray_remote_args.get("num_gpus", required_num_gpus)
391+
memory = ray_remote_args.get("memory", required_memory)
392+
393+
required_resources: Dict[str, float] = {}
394+
if num_cpus > 0:
395+
required_resources["CPU"] = float(num_cpus)
396+
if num_gpus > 0:
397+
required_resources["GPU"] = float(num_gpus)
398+
if memory > 0:
399+
required_resources["memory"] = float(memory)
400+
401+
# Include any additional custom resources requested.
402+
custom_resources = ray_remote_args.get("resources", {})
403+
for name, amount in custom_resources.items():
404+
if amount is None:
405+
continue
406+
try:
407+
amount = float(amount)
408+
except (TypeError, ValueError) as err:
409+
raise ValueError(f"Invalid resource amount for '{name}': {amount}") from err
410+
if amount > 0:
411+
required_resources[name] = amount
412+
413+
# If there are no positive resource requirements, the task can run on a
414+
# zero-resource head node (e.g., num_cpus=0 opt-out), so nothing to check.
415+
if not required_resources:
416+
return
417+
418+
head_node = next(
419+
(
420+
node
421+
for node in ray.nodes()
422+
if node.get("Alive")
423+
and "node:__internal_head__" in node.get("Resources", {})
424+
),
425+
None,
426+
)
427+
if not head_node:
428+
# The head node metadata is unavailable (e.g., during shutdown). Fall back
429+
# to the default behavior and let Ray surface its own error.
430+
return
431+
432+
# Build a map of required vs available resources on the head node.
433+
head_resources: Dict[str, float] = head_node.get("Resources", {})
434+
# Map: resource name -> (required, available).
435+
insufficient: Dict[str, Tuple[float, float]] = {}
436+
for name, req in required_resources.items():
437+
avail = head_resources.get(name, 0.0)
438+
if avail < req:
439+
insufficient[name] = (req, avail)
440+
441+
# If nothing is below the required amount, we are good to proceed.
442+
if not insufficient:
443+
return
444+
445+
details = "; ".join(
446+
f"{name} required {req:g} but head has {avail:g}"
447+
for name, (req, avail) in insufficient.items()
448+
)
449+
450+
raise ValueError(
451+
f"{op_name} must run on the head node (e.g., for local:// paths), "
452+
f"but the head node doesn't have enough resources: {details}. "
453+
"Add resources to the head node, switch to a shared filesystem instead "
454+
"of local://, or set the resource requests on this operation to 0 "
455+
"(for example, num_cpus=0) so it can run without head resources."
456+
)
457+
458+
371459
def _truncated_repr(obj: Any) -> str:
372460
"""Utility to return a truncated object representation for error messages."""
373461
msg = str(obj)

python/ray/data/dataset.py

Lines changed: 6 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -89,6 +89,7 @@
8989
from ray.data._internal.util import (
9090
AllToAllAPI,
9191
ConsumptionAPI,
92+
_validate_head_node_resources_for_local_scheduling,
9293
_validate_rows_per_file_args,
9394
get_compute_strategy,
9495
merge_resources_to_ray_remote_args,
@@ -5351,6 +5352,11 @@ def write_datasink(
53515352
soft=False,
53525353
)
53535354

5355+
_validate_head_node_resources_for_local_scheduling(
5356+
ray_remote_args,
5357+
op_name="Writing to a local:// path",
5358+
)
5359+
53545360
plan = self._plan.copy()
53555361
write_op = Write(
53565362
self._logical_plan.dag,

python/ray/data/read_api.py

Lines changed: 7 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -76,6 +76,7 @@
7676
from ray.data._internal.tensor_extensions.utils import _create_possibly_ragged_ndarray
7777
from ray.data._internal.util import (
7878
_autodetect_parallelism,
79+
_validate_head_node_resources_for_local_scheduling,
7980
get_table_block_metadata_schema,
8081
merge_resources_to_ray_remote_args,
8182
ndarray_to_block,
@@ -440,6 +441,12 @@ def read_datasource(
440441
ray_remote_args,
441442
)
442443

444+
if not datasource.supports_distributed_reads:
445+
_validate_head_node_resources_for_local_scheduling(
446+
ray_remote_args,
447+
op_name="Reading from a local:// path",
448+
)
449+
443450
datasource_or_legacy_reader = _get_datasource_or_legacy_reader(
444451
datasource,
445452
ctx,

python/ray/data/tests/test_consumption.py

Lines changed: 36 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -743,6 +743,42 @@ def check_dataset_is_local(ds):
743743
).materialize()
744744

745745

746+
def test_read_local_scheme_zero_head_resources(ray_start_cluster, tmp_path):
747+
cluster = ray_start_cluster
748+
cluster.add_node(num_cpus=0)
749+
cluster.wait_for_nodes()
750+
751+
ray.shutdown()
752+
ray.init(address=cluster.address)
753+
754+
cluster.add_node(num_cpus=1)
755+
cluster.wait_for_nodes()
756+
757+
csv_path = tmp_path / "data.csv"
758+
pd.DataFrame({"a": [1, 2]}).to_csv(csv_path, index=False)
759+
760+
with pytest.raises(ValueError, match=r"local://.*head node"):
761+
ray.data.read_csv(f"local://{csv_path}").materialize()
762+
763+
764+
def test_write_local_scheme_zero_head_resources(ray_start_cluster, tmp_path):
765+
cluster = ray_start_cluster
766+
cluster.add_node(num_cpus=0)
767+
cluster.wait_for_nodes()
768+
769+
ray.shutdown()
770+
ray.init(address=cluster.address)
771+
772+
cluster.add_node(num_cpus=1)
773+
cluster.wait_for_nodes()
774+
775+
output_dir = tmp_path / "local_out"
776+
ds = ray.data.range(10)
777+
778+
with pytest.raises(ValueError, match=r"local://.*head node"):
779+
ds.write_parquet(f"local://{output_dir}")
780+
781+
746782
class FlakyCSVDatasource(CSVDatasource):
747783
def __init__(self, paths, **csv_datasource_kwargs):
748784
super().__init__(paths, **csv_datasource_kwargs)

0 commit comments

Comments
 (0)