Class ExecutionConfig

  • All Implemented Interfaces:
    Serializable, Archiveable<ArchivedExecutionConfig>

    @Public
    public class ExecutionConfig
    extends Object
    implements Serializable, Archiveable<ArchivedExecutionConfig>
    A config to define the behavior of the program execution. It allows to define (among other options) the following settings:
    • The default parallelism of the program, i.e., how many parallel tasks to use for all functions that do not define a specific value directly.
    • The number of retries in the case of failed executions.
    • The delay between execution retries.
    • Enabling or disabling the "closure cleaner". The closure cleaner pre-processes the implementations of functions. In case they are (anonymous) inner classes, it removes unused references to the enclosing class to fix certain serialization-related problems and to reduce the size of the closure.
    • The config allows to register types and serializers to increase the efficiency of handling generic types and POJOs. This is usually only needed when the functions return not only the types declared in their signature, but also subclasses of those types.
    See Also:
    Serialized Form
    • Field Detail

      • PARALLELISM_AUTO_MAX

        @Deprecated
        public static final int PARALLELISM_AUTO_MAX
        Deprecated.
        The constant to use for the parallelism, if the system should use the number of currently available slots.
        See Also:
        Constant Field Values
      • PARALLELISM_DEFAULT

        public static final int PARALLELISM_DEFAULT
        The flag value indicating use of the default parallelism. This value can be used to reset the parallelism back to the default state.
        See Also:
        Constant Field Values
      • PARALLELISM_UNKNOWN

        public static final int PARALLELISM_UNKNOWN
        The flag value indicating an unknown or unset parallelism. This value is not a valid parallelism and indicates that the parallelism should remain unchanged.
        See Also:
        Constant Field Values
    • Constructor Detail

      • ExecutionConfig

        public ExecutionConfig()
    • Method Detail

      • enableClosureCleaner

        public ExecutionConfig enableClosureCleaner()
        Enables the ClosureCleaner. This analyzes user code functions and sets fields to null that are not used. This will in most cases make closures or anonymous inner classes serializable that where not serializable due to some Scala or Java implementation artifact. User code must be serializable because it needs to be sent to worker nodes.
      • isClosureCleanerEnabled

        public boolean isClosureCleanerEnabled()
        Returns whether the ClosureCleaner is enabled.
        See Also:
        enableClosureCleaner()
      • setAutoWatermarkInterval

        @PublicEvolving
        public ExecutionConfig setAutoWatermarkInterval​(long interval)
        Sets the interval of the automatic watermark emission. Watermarks are used throughout the streaming system to keep track of the progress of time. They are used, for example, for time based windowing.

        Setting an interval of 0 will disable periodic watermark emission.

        Parameters:
        interval - The interval between watermarks in milliseconds.
      • setLatencyTrackingInterval

        @PublicEvolving
        public ExecutionConfig setLatencyTrackingInterval​(long interval)
        Interval for sending latency tracking marks from the sources to the sinks. Flink will send latency tracking marks from the sources at the specified interval.

        Setting a tracking interval <= 0 disables the latency tracking.

        Parameters:
        interval - Interval in milliseconds.
      • getLatencyTrackingInterval

        @PublicEvolving
        public long getLatencyTrackingInterval()
        Returns the latency tracking interval.
        Returns:
        The latency tracking interval in milliseconds
      • isLatencyTrackingConfigured

        @Internal
        public boolean isLatencyTrackingConfigured()
      • isPeriodicMaterializeEnabled

        @Internal
        public boolean isPeriodicMaterializeEnabled()
      • enablePeriodicMaterialize

        @Internal
        public void enablePeriodicMaterialize​(boolean enabled)
      • getPeriodicMaterializeIntervalMillis

        @Internal
        public long getPeriodicMaterializeIntervalMillis()
      • setPeriodicMaterializeIntervalMillis

        @Internal
        public void setPeriodicMaterializeIntervalMillis​(Duration periodicMaterializeInterval)
      • getMaterializationMaxAllowedFailures

        @Internal
        public int getMaterializationMaxAllowedFailures()
      • setMaterializationMaxAllowedFailures

        @Internal
        public void setMaterializationMaxAllowedFailures​(int materializationMaxAllowedFailures)
      • getParallelism

        public int getParallelism()
        Gets the parallelism with which operation are executed by default. Operations can individually override this value to use a specific parallelism.

        Other operations may need to run with a different parallelism - for example calling a reduce operation over the entire data set will involve an operation that runs with a parallelism of one (the final reduce to the single result value).

        Returns:
        The parallelism used by operations, unless they override that value. This method returns PARALLELISM_DEFAULT if the environment's default parallelism should be used.
      • setParallelism

        public ExecutionConfig setParallelism​(int parallelism)
        Sets the parallelism for operations executed through this environment. Setting a parallelism of x here will cause all operators (such as join, map, reduce) to run with x parallel instances.

        This method overrides the default parallelism for this environment. The local execution environment uses by default a value equal to the number of hardware contexts (CPU cores / threads). When executing the program via the command line client from a JAR file, the default parallelism is the one configured for that setup.

        Parameters:
        parallelism - The parallelism to use
      • resetParallelism

        @Internal
        public void resetParallelism()
      • getMaxParallelism

        @PublicEvolving
        public int getMaxParallelism()
        Gets the maximum degree of parallelism defined for the program.

        The maximum degree of parallelism specifies the upper limit for dynamic scaling. It also defines the number of key groups used for partitioned state.

        Returns:
        Maximum degree of parallelism
      • setMaxParallelism

        @PublicEvolving
        public void setMaxParallelism​(int maxParallelism)
        Sets the maximum degree of parallelism defined for the program.

        The maximum degree of parallelism specifies the upper limit for dynamic scaling. It also defines the number of key groups used for partitioned state.

        Parameters:
        maxParallelism - Maximum degree of parallelism to be used for the program.
      • getTaskCancellationInterval

        public long getTaskCancellationInterval()
        Gets the interval (in milliseconds) between consecutive attempts to cancel a running task.
      • setTaskCancellationInterval

        public ExecutionConfig setTaskCancellationInterval​(long interval)
        Sets the configuration parameter specifying the interval (in milliseconds) between consecutive attempts to cancel a running task.
        Parameters:
        interval - the interval (in milliseconds).
      • getTaskCancellationTimeout

        @PublicEvolving
        public long getTaskCancellationTimeout()
        Returns the timeout (in milliseconds) after which an ongoing task cancellation leads to a fatal TaskManager error.

        The value 0 means that the timeout is disabled. In this case a stuck cancellation will not lead to a fatal error.

      • setTaskCancellationTimeout

        @PublicEvolving
        public ExecutionConfig setTaskCancellationTimeout​(long timeout)
        Sets the timeout (in milliseconds) after which an ongoing task cancellation is considered failed, leading to a fatal TaskManager error.

        The cluster default is configured via TaskManagerOptions.TASK_CANCELLATION_TIMEOUT.

        The value 0 disables the timeout. In this case a stuck cancellation will not lead to a fatal error.

        Parameters:
        timeout - The task cancellation timeout (in milliseconds).
      • enableAutoGeneratedUIDs

        public void enableAutoGeneratedUIDs()
        Enables the Flink runtime to auto-generate UID's for operators.
        See Also:
        disableAutoGeneratedUIDs()
      • disableAutoGeneratedUIDs

        public void disableAutoGeneratedUIDs()
        Disables auto-generated UIDs. Forces users to manually specify UIDs on DataStream applications.

        It is highly recommended that users specify UIDs before deploying to production since they are used to match state in savepoints to operators in a job. Because auto-generated ID's are likely to change when modifying a job, specifying custom IDs allow an application to evolve overtime without discarding state.

      • enableObjectReuse

        public ExecutionConfig enableObjectReuse()
        Enables reusing objects that Flink internally uses for deserialization and passing data to user-code functions. Keep in mind that this can lead to bugs when the user-code function of an operation is not aware of this behaviour.
      • disableObjectReuse

        public ExecutionConfig disableObjectReuse()
        Disables reusing objects that Flink internally uses for deserialization and passing data to user-code functions. @see #enableObjectReuse()
      • isObjectReuseEnabled

        public boolean isObjectReuseEnabled()
        Returns whether object reuse has been enabled or disabled. @see #enableObjectReuse()
      • setGlobalJobParameters

        public void setGlobalJobParameters​(ExecutionConfig.GlobalJobParameters globalJobParameters)
        Register a custom, serializable user configuration object.
        Parameters:
        globalJobParameters - Custom user configuration object
      • isUseSnapshotCompression

        public boolean isUseSnapshotCompression()
      • setUseSnapshotCompression

        public void setUseSnapshotCompression​(boolean useSnapshotCompression)
      • getAsyncInflightRecordsLimit

        @Experimental
        public int getAsyncInflightRecordsLimit()
      • getAsyncStateBufferSize

        @Experimental
        public int getAsyncStateBufferSize()
      • getAsyncStateBufferTimeout

        @Experimental
        public long getAsyncStateBufferTimeout()
      • hashCode

        public int hashCode()
        Overrides:
        hashCode in class Object
      • configure

        public void configure​(ReadableConfig configuration,
                              ClassLoader classLoader)
        Sets all relevant options contained in the ReadableConfig such as e.g. PipelineOptions.CLOSURE_CLEANER_LEVEL.

        It will change the value of a setting only if a corresponding option was set in the configuration. If a key is not present, the current value of a field will remain untouched.

        Parameters:
        configuration - a configuration to read the values from
        classLoader - a class loader to use when loading classes