-
Notifications
You must be signed in to change notification settings - Fork 5.8k
Commit
This commit does not belong to any branch on this repository, and may belong to a fork outside of the repository.
[core] Support generators to allow tasks to return a dynamic number o…
…f objects (#28291) This adds support for tasks that need to return a dynamic number of objects. When a remote generator function is invoked and num_returns for the task is 1, the worker will dynamically allocate ray.put IDs for these objects and store an ObjectRefGenerator as its return value. This allows the worker to choose how many objects to return and to keep heap memory low, since it does not need to keep all objects in memory simultaneously. Unlike normal ray.put(), we assign the task caller as the owner of the object. This is to improve fault tolerance, as the owner can recover dynamically generated objects through the normal lineage reconstruction codepath. The main complication has to do with notifying the task caller that it owns these objects. We do this in two places, which is necessary because the protocols are asynchronous, so either message can arrive first. When the task reply is received. When the primary raylet subscribes to the eviction notice from the owner. To register the dynamic return, the owner adds the ObjectRef to the ref counter and marks that it is contained in the generator object. Signed-off-by: Stephanie Wang <[email protected]> Co-authored-by: Clark Zinzow <[email protected]> Co-authored-by: Eric Liang <[email protected]>
- Loading branch information
1 parent
014279b
commit 45d7cd2
Showing
58 changed files
with
1,323 additions
and
284 deletions.
There are no files selected for viewing
This file contains bidirectional Unicode text that may be interpreted or compiled differently than what appears below. To review, open the file in an editor that reveals hidden Unicode characters.
Learn more about bidirectional Unicode characters
This file contains bidirectional Unicode text that may be interpreted or compiled differently than what appears below. To review, open the file in an editor that reveals hidden Unicode characters.
Learn more about bidirectional Unicode characters
This file contains bidirectional Unicode text that may be interpreted or compiled differently than what appears below. To review, open the file in an editor that reveals hidden Unicode characters.
Learn more about bidirectional Unicode characters
This file contains bidirectional Unicode text that may be interpreted or compiled differently than what appears below. To review, open the file in an editor that reveals hidden Unicode characters.
Learn more about bidirectional Unicode characters
Original file line number | Diff line number | Diff line change |
---|---|---|
@@ -0,0 +1,71 @@ | ||
# __program_start__ | ||
import ray | ||
from ray import ObjectRefGenerator | ||
|
||
# fmt: off | ||
# __dynamic_generator_start__ | ||
import numpy as np | ||
|
||
|
||
@ray.remote(num_returns="dynamic") | ||
def split(array, chunk_size): | ||
while len(array) > 0: | ||
yield array[:chunk_size] | ||
array = array[chunk_size:] | ||
|
||
|
||
array_ref = ray.put(np.zeros(np.random.randint(1000_000))) | ||
block_size = 1000 | ||
|
||
# Returns an ObjectRef[ObjectRefGenerator]. | ||
dynamic_ref = split.remote(array_ref, block_size) | ||
print(dynamic_ref) | ||
# ObjectRef(c8ef45ccd0112571ffffffffffffffffffffffff0100000001000000) | ||
|
||
i = -1 | ||
ref_generator = ray.get(dynamic_ref) | ||
print(ref_generator) | ||
# <ray._raylet.ObjectRefGenerator object at 0x7f7e2116b290> | ||
for i, ref in enumerate(ref_generator): | ||
# Each ObjectRefGenerator iteration returns an ObjectRef. | ||
assert len(ray.get(ref)) <= block_size | ||
num_blocks_generated = i + 1 | ||
array_size = len(ray.get(array_ref)) | ||
assert array_size <= num_blocks_generated * block_size | ||
print(f"Split array of size {array_size} into {num_blocks_generated} blocks of " | ||
f"size {block_size} each.") | ||
# Split array of size 63153 into 64 blocks of size 1000 each. | ||
|
||
# NOTE: The dynamic_ref points to the generated ObjectRefs. Make sure that this | ||
# ObjectRef goes out of scope so that Ray can garbage-collect the internal | ||
# ObjectRefs. | ||
del dynamic_ref | ||
# __dynamic_generator_end__ | ||
# fmt: on | ||
|
||
|
||
# fmt: off | ||
# __dynamic_generator_pass_start__ | ||
@ray.remote | ||
def get_size(ref_generator : ObjectRefGenerator): | ||
print(ref_generator) | ||
num_elements = 0 | ||
for ref in ref_generator: | ||
array = ray.get(ref) | ||
assert len(array) <= block_size | ||
num_elements += len(array) | ||
return num_elements | ||
|
||
|
||
# Returns an ObjectRef[ObjectRefGenerator]. | ||
dynamic_ref = split.remote(array_ref, block_size) | ||
assert array_size == ray.get(get_size.remote(dynamic_ref)) | ||
# (get_size pid=1504184) <ray._raylet.ObjectRefGenerator object at 0x7f81c4250ad0> | ||
|
||
# This also works, but should be avoided because you have to call an additional | ||
# `ray.get`, which blocks the driver. | ||
ref_generator = ray.get(dynamic_ref) | ||
assert array_size == ray.get(get_size.remote(ref_generator)) | ||
# (get_size pid=1504184) <ray._raylet.ObjectRefGenerator object at 0x7f81c4251b50> | ||
# __dynamic_generator_pass_end__ | ||
# fmt: on |
This file contains bidirectional Unicode text that may be interpreted or compiled differently than what appears below. To review, open the file in an editor that reveals hidden Unicode characters.
Learn more about bidirectional Unicode characters
This file contains bidirectional Unicode text that may be interpreted or compiled differently than what appears below. To review, open the file in an editor that reveals hidden Unicode characters.
Learn more about bidirectional Unicode characters
This file contains bidirectional Unicode text that may be interpreted or compiled differently than what appears below. To review, open the file in an editor that reveals hidden Unicode characters.
Learn more about bidirectional Unicode characters
Original file line number | Diff line number | Diff line change |
---|---|---|
@@ -0,0 +1,86 @@ | ||
.. _generators: | ||
|
||
Generators | ||
========== | ||
|
||
Python generators are functions that behave like an iterator, yielding one | ||
value per iteration. Ray supports remote generators for two use cases: | ||
|
||
1. To reduce max heap memory usage when returning multiple values from a remote | ||
function. See the :ref:`design pattern guide <generator-pattern>` for an | ||
example. | ||
2. When the number of return values is set dynamically by the remote function | ||
instead of by the caller. | ||
|
||
Remote generators can be used in both actor and non-actor tasks. | ||
|
||
`num_returns` set by the task caller | ||
------------------------------------ | ||
|
||
Where possible, the caller should set the remote function's number of return values using ``@ray.remote(num_returns=x)`` or ``foo.options(num_returns=x).remote()``. | ||
Ray will return this many ``ObjectRefs`` to the caller. | ||
The remote task should then return the same number of values, usually as a tuple or list. | ||
Compared to setting the number of return values dynamically, this adds less complexity to user code and less performance overhead, as Ray will know exactly how many ``ObjectRefs`` to return to the caller ahead of time. | ||
|
||
Without changing the caller's syntax, we can also use a remote generator function to return the values iteratively. | ||
The generator should return the same number of return values specified by the caller, and these will be stored one at a time in Ray's object store. | ||
An error will be returned for generators that return a different number of values from the one specified by the caller. | ||
|
||
For example, we can swap the following code that returns a list of return values: | ||
|
||
.. literalinclude:: ../doc_code/pattern_generators.py | ||
:language: python | ||
:start-after: __large_values_start__ | ||
:end-before: __large_values_end__ | ||
|
||
for this code, which uses a generator function: | ||
|
||
.. literalinclude:: ../doc_code/pattern_generators.py | ||
:language: python | ||
:start-after: __large_values_generator_start__ | ||
:end-before: __large_values_generator_end__ | ||
|
||
The advantage of doing so is that the generator function does not need to hold all of its return values in memory at once. | ||
It can return the arrays one at a time to reduce memory pressure. | ||
|
||
`num_returns` set by the task executor | ||
-------------------------------------- | ||
|
||
In some cases, the caller may not know the number of return values to expect from a remote function. | ||
For example, suppose we want to write a task that breaks up its argument into equal-size chunks and returns these. | ||
We may not know the size of the argument until we execute the task, so we don't know the number of return values to expect. | ||
|
||
In these cases, we can use a remote generator function that returns a *dynamic* number of values. | ||
To use this feature, set ``num_returns="dynamic"`` in the ``@ray.remote`` decorator or the remote function's ``.options()``. | ||
Then, when invoking the remote function, Ray will return a *single* ``ObjectRef`` that will get populated with an ``ObjectRefGenerator`` when the task completes. | ||
The ``ObjectRefGenerator`` can be used to iterate over a list of ``ObjectRefs`` containing the actual values returned by the task. | ||
|
||
.. note:: ``num_returns="dynamic"`` is currently an experimental API in v2.1+. | ||
|
||
.. literalinclude:: ../doc_code/generator.py | ||
:language: python | ||
:start-after: __dynamic_generator_start__ | ||
:end-before: __dynamic_generator_end__ | ||
|
||
We can also pass the ``ObjectRef`` returned by a task with ``num_returns="dynamic"`` to another task. The task will receive the ``ObjectRefGenerator``, which it can use to iterate over the task's return values. Similarly, you can also pass an ``ObjectRefGenerator`` as a task argument. | ||
|
||
.. literalinclude:: ../doc_code/generator.py | ||
:language: python | ||
:start-after: __dynamic_generator_pass_start__ | ||
:end-before: __dynamic_generator_pass_end__ | ||
|
||
Limitations | ||
----------- | ||
|
||
Although a generator function creates ``ObjectRefs`` one at a time, currently Ray will not schedule dependent tasks until the entire task is complete and all values have been created. This is similar to the semantics used by tasks that return multiple values as a list. | ||
|
||
``num_returns="dynamic"`` is not yet supported for actor tasks. | ||
|
||
If a generator function raises an exception before yielding all its values, all values returned by the generator will be replaced by the exception traceback, including values that were already successfully yielded. | ||
If the task was called with ``num_returns="dynamic"``, the exception will be stored in the ``ObjectRef`` returned by the task instead of the usual ``ObjectRefGenerator``. | ||
|
||
Note that there is currently a known bug where exceptions will not be propagated for generators that yield objects in Ray's shared-memory object store before erroring. In this case, these objects will still be accessible through the returned ``ObjectRefs`` and you may see an error like the following: | ||
|
||
.. code-block:: text | ||
$ ERROR worker.py:754 -- Generator threw exception after returning partial values in the object store, error may be unhandled. |
This file contains bidirectional Unicode text that may be interpreted or compiled differently than what appears below. To review, open the file in an editor that reveals hidden Unicode characters.
Learn more about bidirectional Unicode characters
This file contains bidirectional Unicode text that may be interpreted or compiled differently than what appears below. To review, open the file in an editor that reveals hidden Unicode characters.
Learn more about bidirectional Unicode characters
This file contains bidirectional Unicode text that may be interpreted or compiled differently than what appears below. To review, open the file in an editor that reveals hidden Unicode characters.
Learn more about bidirectional Unicode characters
This file contains bidirectional Unicode text that may be interpreted or compiled differently than what appears below. To review, open the file in an editor that reveals hidden Unicode characters.
Learn more about bidirectional Unicode characters
This file contains bidirectional Unicode text that may be interpreted or compiled differently than what appears below. To review, open the file in an editor that reveals hidden Unicode characters.
Learn more about bidirectional Unicode characters
Oops, something went wrong.