ray/doc/source/ray-core/ray-dag.rst

Ignoring revisions in .git-blame-ignore-revs. Click here to bypass and see the normal blame view.

159 lines
5.3 KiB
ReStructuredText
Raw Normal View History

.. _ray-dag-guide:
Building Computation Graphs with Ray DAG API
============================================
With ``ray.remote`` you have the flexibility of running an application where
computation is executed remotely at runtime. For a ``ray.remote`` decorated
class or function, you can also use ``.bind`` on the body to build a static
computation graph.
.. note::
Ray DAG is designed to be developer facing API where recommended use cases
are
1) Locally iterate and test your application authored by higher level libraries.
2) Build libraries on top of the Ray DAG APIs.
When ``.bind()`` is called on a ``ray.remote`` decorated class or function, it will
generate an intermediate representation (IR) node that act as backbone and
building blocks of the DAG that is statically holding the computation graph
together, where each IR node is resolved to value at execution time with
respect to their topological order.
The IR node can also be assigned to a variable and passed into other nodes as
arguments.
Ray DAG with functions
----------------------
The IR node generated by ``.bind()`` on a ``ray.remote`` decorated function is
executed as a Ray Task upon execution which will be solved to the task output.
This example shows how to build a chain of functions where each node can be
executed as root node while iterating, or used as input args or kwargs of other
functions to form more complex DAGs.
Any IR node can be executed directly ``dag_node.execute()`` that acts as root
of the DAG, where all other non-reachable nodes from the root will be igored.
.. tabbed:: Python
.. code-block:: python
# `ray start --head` has been run to launch a local cluster
import ray
@ray.remote
def func(src, inc=1):
return src + inc
a_ref = func.bind(1, inc=2)
assert ray.get(a_ref.execute()) == 3 # 1 + 2 = 3
b_ref = func.bind(a_ref, inc=3)
assert ray.get(b_ref.execute()) == 6 # (1 + 2) + 3 = 6
c_ref = func.bind(b_ref, inc=a_ref)
assert ray.get(c_ref.execute()) == 9 # ((1 + 2) + 3) + (1 + 2) = 9
Ray DAG with classes and class methods
--------------------------------------
The IR node generated by ``.bind()`` on a ``ray.remote`` decorated class is
executed as a Ray Actor upon execution. The Actor will be instantiated every
time the node is executed, and the classmethod calls can form a chain of
function calls specific to the parent actor instance.
DAG IR nodes generated from a function, class or classmethod can be combined
together to form a DAG.
.. tabbed:: Python
.. code-block:: python
# `ray start --head` has been run to launch a local cluster
import ray
@ray.remote
class Actor:
def __init__(self, init_value):
self.i = init_value
def inc(self, x):
self.i += x
def get(self):
return self.i
a1 = Actor.bind(10) # Instantiate Actor with init_value 10.
val = a1.get.bind() # ClassMethod that returns value from get() from
# the actor created.
assert ray.get(val.execute()) == 10
@ray.remote
def combine(x, y):
return x + y
a2 = Actor.bind(10) # Instantiate another Actor with init_value 10.
a1.inc.bind(2) # Call inc() on the actor created with increment of 2.
a1.inc.bind(4) # Call inc() on the actor created with increment of 4.
a2.inc.bind(6) # Call inc() on the actor created with increment of 6.
# Combine outputs from a1.get() and a2.get()
dag = combine.bind(a1.get.bind(), a2.get.bind())
# a1 + a2 + inc(2) + inc(4) + inc(6)
# 10 + (10 + ( 2 + 4 + 6)) = 32
assert ray.get(dag.execute()) == 32
Ray DAG with custom InputNode
-----------------------------
``InputNode`` is the singleton node of a DAG that represents user input value at
runtime. It should be used within a context manager with no args, and called
as args of ``dag_node.execute()``
.. tabbed:: Python
.. code-block:: python
# `ray start --head` has been run to launch a local cluster
import ray
from ray.dag.input_node import InputNode
@ray.remote
def a(user_input):
return user_input * 2
@ray.remote
def b(user_input):
return user_input + 1
@ray.remote
def c(x, y):
return x + y
with InputNode() as dag_input:
a_ref = a.bind(dag_input)
b_ref = b.bind(dag_input)
dag = c.bind(a_ref, b_ref)
# a(2) + b(2) = c
# (2 * 2) + (2 * 1)
assert ray.get(dag.execute(2)) == 7
# a(3) + b(3) = c
# (3 * 2) + (3 * 1)
assert ray.get(dag.execute(3)) == 10
More Resources
--------------
You can find more application patterns and examples in the following resources
from other Ray libraries built on top of Ray DAG API with same mechanism.
| `Visualization of DAGs <https://docs.ray.io/en/master/serve/deployment-graph/visualize_dag_during_development.html>`_
| `DAG Cookbook and patterns <https://docs.ray.io/en/master/serve/deployment-graph.html#patterns>`_
| `Serve Deployment Graph's original REP <https://github.com/ray-project/enhancements/blob/main/reps/2022-03-08-serve_pipeline.md>`_