Back to Ray

Lazy Computation Graphs with the Ray DAG API

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

1.13.14.6 KB
Original Source

.. _ray-dag-guide:

Lazy Computation Graphs with the 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 a 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.

.. note::

Ray has introduced an experimental API for high-performance workloads that is
especially well suited for applications using multiple GPUs. This API is built on top of
the Ray DAG API.

See :ref:`Ray Compiled Graph <ray-compiled-graph>` for more details.

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 ignored.

.. tab-set::

.. tab-item:: Python

    .. literalinclude:: ./doc_code/ray-dag.py
      :language: python
      :start-after: __dag_tasks_begin__
      :end-before: __dag_tasks_end__

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.

.. tab-set::

.. tab-item:: Python

    .. literalinclude:: ./doc_code/ray-dag.py
      :language: python
      :start-after: __dag_actors_begin__
      :end-before: __dag_actors_end__

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()

.. tab-set::

.. tab-item:: Python

    .. literalinclude:: ./doc_code/ray-dag.py
      :language: python
      :start-after: __dag_input_node_begin__
      :end-before: __dag_input_node_end__

Ray DAG with multiple MultiOutputNode

MultiOutputNode is useful when you have more than 1 output from a DAG. dag_node.execute() returns a list of Ray object references passed to MultiOutputNode. The below example shows the multi output node of 2 outputs.

.. tab-set::

.. tab-item:: Python

    .. literalinclude:: ./doc_code/ray-dag.py
      :language: python
      :start-after: __dag_multi_output_node_begin__
      :end-before: __dag_multi_output_node_end__

Reuse Ray Actors in DAGs

Actors can be a part of the DAG definition with the Actor.bind() API. However, when a DAG finishes execution, Ray kills Actors created with bind.

You can avoid killing your Actors whenever DAG finishes by creating Actors with Actor.remote().

.. tab-set::

.. tab-item:: Python

    .. literalinclude:: ./doc_code/ray-dag.py
      :language: python
      :start-after: __dag_actor_reuse_begin__
      :end-before: __dag_actor_reuse_end__

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 the same mechanism.

| Ray Serve Compositions of Models <https://docs.ray.io/en/master/serve/model_composition.html>_ | Visualization of Ray Compiled Graph <https://docs.ray.io/en/latest/ray-core/compiled-graph/profiling.html#visualization>_