Using the Direct Runner
- Java SDK
- Python SDK
The Direct Runner executes pipelines on your machine and is designed to validate that pipelines adhere to the Apache Beam model as closely as possible. Instead of focusing on efficient pipeline execution, the Direct Runner performs additional checks to ensure that users do not rely on semantics that are not guaranteed by the model. Some of these checks include:
- enforcing immutability of elements
- enforcing encodability of elements
- elements are processed in an arbitrary order at all points
- serialization of user functions (
DoFn
,CombineFn
, etc.) See Serializability of DoFns for details.
Using the Direct Runner for testing and development helps ensure that pipelines are robust across different Beam runners. In addition, debugging failed runs can be a non-trivial task when a pipeline executes on a remote cluster. Instead, it is often faster and simpler to perform local unit testing on your pipeline code. Unit testing your pipeline locally also allows you to use your preferred local debugging tools.
Here are some resources with information about how to test your pipelines.
- Test Your Pipeline
- Testing Unbounded Pipelines in Apache Beam talks about the use of Java classes PAssert and TestStream to test your pipelines.
- The Apache Beam WordCount Walkthrough contains an example of logging and testing a pipeline with PAssert.
- The Apache Beam WordCount Walkthrough contains an example of logging and testing a pipeline with
assert_that
.
The Direct Runner is not designed for production pipelines, because it’s optimized for correctness rather than performance. The Direct Runner must fit all user data in memory, whereas the Flink and Spark runners can spill data to disk if it doesn’t fit in memory. Consequently, Flink and Spark runners are able to run larger pipelines and are better suited to production workloads.
Direct Runner prerequisites and setup
Specify your dependency
When using Java, you must specify your dependency on the Direct Runner in your pom.xml
.
This section is not applicable to the Beam SDK for Python.
Pipeline options for the Direct Runner
For general instructions on how to set pipeline options, see the programming guide.
When executing your pipeline from the command-line, set runner
to direct
or DirectRunner
. The default values for the other pipeline options are generally sufficient.
See the reference documentation for the
DirectOptions
DirectOptions
interface for defaults and additional pipeline configuration options.
Additional information and caveats
Memory considerations
Local execution is limited by the memory available in your local environment. It is highly recommended that you run your pipeline with data sets small enough to fit in local memory. You can create a small in-memory data set using a Create
Create
transform, or you can use a Read
Read
transform to work with small local or remote files.
Streaming execution
Streaming support for Python DirectRunner is limited. For known issues, see: https://github.com/apache/beam/issues/24528.
If your pipeline uses an unbounded data source or sink, you must set the streaming
option to true
.
Parallel execution
Python FnApiRunner supports multi-threading and multi-processing mode.
Setting parallelism
The number of worker threads is defined by the targetParallelism
pipeline option.
By default, targetParallelism
is the greater of the number of available processors and 3.
Number of threads or subprocesses is defined by setting the direct_num_workers
pipeline option.
From 2.22.0, direct_num_workers = 0
is supported. When direct_num_workers
is set to 0, it will set the number of threads/subprocess to the number of cores of the machine where the pipeline is running.
Setting running mode
In Beam 2.19.0 and newer, you can use the direct_running_mode
pipeline option to set the running mode.
direct_running_mode
can be one of ['in_memory'
, 'multi_threading'
, 'multi_processing'
].
in_memory: Runner and workers’ communication happens in memory (not through gRPC). This is a default mode.
multi_threading: Runner and workers communicate through gRPC and each worker runs in a thread.
multi_processing: Runner and workers communicate through gRPC and each worker runs in a subprocess.
Before deploying pipeline to remote runner
While testing on the direct runner is convenient, it can still behave differently from remote runners beyond Beam model semantics, especially for runtime environment related issues. In general, it is recommended to test your pipeline on targeted remote runner in small scale before fully deploying into production.
Last updated on 2024/12/20
Have you found everything you were looking for?
Was it all useful and clear? Is there anything that you would like to change? Let us know!