@Public public class ExecutionConfig extends Object implements Serializable, Archiveable<ArchivedExecutionConfig>
ExecutionMode
of the program: Batch or Pipelined. The default execution mode is
ExecutionMode.PIPELINED
Modifier and Type | Class and Description |
---|---|
static class |
ExecutionConfig.ClosureCleanerLevel
Configuration settings for the closure cleaner.
|
static class |
ExecutionConfig.GlobalJobParameters
Abstract class for a custom user configuration object registered at the execution config.
|
static class |
ExecutionConfig.SerializableSerializer<T extends com.esotericsoftware.kryo.Serializer<?> & Serializable> |
Modifier and Type | Field and Description |
---|---|
static int |
PARALLELISM_AUTO_MAX
Deprecated.
|
static int |
PARALLELISM_DEFAULT
The flag value indicating use of the default parallelism.
|
static int |
PARALLELISM_UNKNOWN
The flag value indicating an unknown or unset parallelism.
|
Constructor and Description |
---|
ExecutionConfig() |
Modifier and Type | Method and Description |
---|---|
void |
addDefaultKryoSerializer(Class<?> type,
Class<? extends com.esotericsoftware.kryo.Serializer<?>> serializerClass)
Adds a new Kryo default serializer to the Runtime.
|
<T extends com.esotericsoftware.kryo.Serializer<?> & Serializable> |
addDefaultKryoSerializer(Class<?> type,
T serializer)
Adds a new Kryo default serializer to the Runtime.
|
ArchivedExecutionConfig |
archive() |
boolean |
canEqual(Object obj) |
void |
configure(ReadableConfig configuration,
ClassLoader classLoader)
Sets all relevant options contained in the
ReadableConfig such as e.g. |
void |
disableAutoGeneratedUIDs()
Disables auto-generated UIDs.
|
void |
disableAutoTypeRegistration()
Control whether Flink is automatically registering all types in the user programs with Kryo.
|
ExecutionConfig |
disableClosureCleaner()
Disables the ClosureCleaner.
|
void |
disableForceAvro()
Disables the Apache Avro serializer as the forced serializer for POJOs.
|
void |
disableForceKryo()
Disable use of Kryo serializer for all POJOs.
|
void |
disableGenericTypes()
Disables the use of generic types (types that would be serialized via Kryo).
|
ExecutionConfig |
disableObjectReuse()
Disables reusing objects that Flink internally uses for deserialization and passing data to
user-code functions.
|
void |
enableAutoGeneratedUIDs()
Enables the Flink runtime to auto-generate UID's for operators.
|
ExecutionConfig |
enableClosureCleaner()
Enables the ClosureCleaner.
|
void |
enableForceAvro()
Forces Flink to use the Apache Avro serializer for POJOs.
|
void |
enableForceKryo()
Force TypeExtractor to use Kryo serializer for POJOS even though we could analyze as POJO.
|
void |
enableGenericTypes()
Enables the use generic types which are serialized via Kryo.
|
ExecutionConfig |
enableObjectReuse()
Enables reusing objects that Flink internally uses for deserialization and passing data to
user-code functions.
|
boolean |
equals(Object obj) |
long |
getAutoWatermarkInterval()
Returns the interval of the automatic watermark emission.
|
ExecutionConfig.ClosureCleanerLevel |
getClosureCleanerLevel()
Returns the configured
ExecutionConfig.ClosureCleanerLevel . |
InputDependencyConstraint |
getDefaultInputDependencyConstraint()
Gets the default input dependency constraint for vertex scheduling.
|
LinkedHashMap<Class<?>,Class<? extends com.esotericsoftware.kryo.Serializer<?>>> |
getDefaultKryoSerializerClasses()
Returns the registered default Kryo Serializer classes.
|
LinkedHashMap<Class<?>,ExecutionConfig.SerializableSerializer<?>> |
getDefaultKryoSerializers()
Returns the registered default Kryo Serializers.
|
ExecutionMode |
getExecutionMode()
Gets the execution mode used to execute the program.
|
long |
getExecutionRetryDelay()
Deprecated.
Should no longer be used because it is subsumed by RestartStrategyConfiguration
|
ExecutionConfig.GlobalJobParameters |
getGlobalJobParameters() |
long |
getLatencyTrackingInterval()
Returns the latency tracking interval.
|
int |
getMaxParallelism()
Gets the maximum degree of parallelism defined for the program.
|
int |
getNumberOfExecutionRetries()
Deprecated.
Should no longer be used because it is subsumed by RestartStrategyConfiguration
|
int |
getParallelism()
Gets the parallelism with which operation are executed by default.
|
LinkedHashSet<Class<?>> |
getRegisteredKryoTypes()
Returns the registered Kryo types.
|
LinkedHashSet<Class<?>> |
getRegisteredPojoTypes()
Returns the registered POJO types.
|
LinkedHashMap<Class<?>,Class<? extends com.esotericsoftware.kryo.Serializer<?>>> |
getRegisteredTypesWithKryoSerializerClasses()
Returns the registered types with their Kryo Serializer classes.
|
LinkedHashMap<Class<?>,ExecutionConfig.SerializableSerializer<?>> |
getRegisteredTypesWithKryoSerializers()
Returns the registered types with Kryo Serializers.
|
RestartStrategies.RestartStrategyConfiguration |
getRestartStrategy()
Returns the restart strategy which has been set for the current job.
|
long |
getTaskCancellationInterval()
Gets the interval (in milliseconds) between consecutive attempts to cancel a running task.
|
long |
getTaskCancellationTimeout()
Returns the timeout (in milliseconds) after which an ongoing task cancellation leads to a
fatal TaskManager error.
|
boolean |
hasAutoGeneratedUIDsEnabled()
Checks whether auto generated UIDs are supported.
|
boolean |
hasGenericTypesDisabled()
Checks whether generic types are supported.
|
int |
hashCode() |
boolean |
isAutoTypeRegistrationDisabled() |
boolean |
isClosureCleanerEnabled()
Returns whether the ClosureCleaner is enabled.
|
boolean |
isForceAvroEnabled()
Returns whether the Apache Avro is the default serializer for POJOs.
|
boolean |
isForceKryoEnabled() |
boolean |
isLatencyTrackingConfigured() |
boolean |
isObjectReuseEnabled()
Returns whether object reuse has been enabled or disabled.
|
boolean |
isUseSnapshotCompression() |
void |
registerKryoType(Class<?> type)
Registers the given type with the serialization stack.
|
void |
registerPojoType(Class<?> type)
Registers the given type with the serialization stack.
|
void |
registerTypeWithKryoSerializer(Class<?> type,
Class<? extends com.esotericsoftware.kryo.Serializer> serializerClass)
Registers the given Serializer via its class as a serializer for the given type at the
KryoSerializer
|
<T extends com.esotericsoftware.kryo.Serializer<?> & Serializable> |
registerTypeWithKryoSerializer(Class<?> type,
T serializer)
Registers the given type with a Kryo Serializer.
|
ExecutionConfig |
setAutoWatermarkInterval(long interval)
Sets the interval of the automatic watermark emission.
|
ExecutionConfig |
setClosureCleanerLevel(ExecutionConfig.ClosureCleanerLevel level)
Configures the closure cleaner.
|
void |
setDefaultInputDependencyConstraint(InputDependencyConstraint inputDependencyConstraint)
Sets the default input dependency constraint for vertex scheduling.
|
void |
setExecutionMode(ExecutionMode executionMode)
Sets the execution mode to execute the program.
|
ExecutionConfig |
setExecutionRetryDelay(long executionRetryDelay)
Deprecated.
This method will be replaced by
setRestartStrategy(org.apache.flink.api.common.restartstrategy.RestartStrategies.RestartStrategyConfiguration) . The RestartStrategies.FixedDelayRestartStrategyConfiguration contains the delay between
successive execution attempts. |
void |
setGlobalJobParameters(ExecutionConfig.GlobalJobParameters globalJobParameters)
Register a custom, serializable user configuration object.
|
ExecutionConfig |
setLatencyTrackingInterval(long interval)
Interval for sending latency tracking marks from the sources to the sinks.
|
void |
setMaxParallelism(int maxParallelism)
Sets the maximum degree of parallelism defined for the program.
|
ExecutionConfig |
setNumberOfExecutionRetries(int numberOfExecutionRetries)
Deprecated.
This method will be replaced by
setRestartStrategy(org.apache.flink.api.common.restartstrategy.RestartStrategies.RestartStrategyConfiguration) . The RestartStrategies.FixedDelayRestartStrategyConfiguration contains the number of
execution retries. |
ExecutionConfig |
setParallelism(int parallelism)
Sets the parallelism for operations executed through this environment.
|
void |
setRestartStrategy(RestartStrategies.RestartStrategyConfiguration restartStrategyConfiguration)
Sets the restart strategy to be used for recovery.
|
ExecutionConfig |
setTaskCancellationInterval(long interval)
Sets the configuration parameter specifying the interval (in milliseconds) between
consecutive attempts to cancel a running task.
|
ExecutionConfig |
setTaskCancellationTimeout(long timeout)
Sets the timeout (in milliseconds) after which an ongoing task cancellation is considered
failed, leading to a fatal TaskManager error.
|
void |
setUseSnapshotCompression(boolean useSnapshotCompression) |
String |
toString() |
@Deprecated public static final int PARALLELISM_AUTO_MAX
public static final int PARALLELISM_DEFAULT
public static final int PARALLELISM_UNKNOWN
public ExecutionConfig enableClosureCleaner()
public ExecutionConfig disableClosureCleaner()
enableClosureCleaner()
public boolean isClosureCleanerEnabled()
enableClosureCleaner()
public ExecutionConfig setClosureCleanerLevel(ExecutionConfig.ClosureCleanerLevel level)
ExecutionConfig.ClosureCleanerLevel
for details on the
different settings.public ExecutionConfig.ClosureCleanerLevel getClosureCleanerLevel()
ExecutionConfig.ClosureCleanerLevel
.@PublicEvolving public ExecutionConfig setAutoWatermarkInterval(long interval)
Setting an interval of 0
will disable periodic watermark emission.
interval
- The interval between watermarks in milliseconds.@PublicEvolving public long getAutoWatermarkInterval()
setAutoWatermarkInterval(long)
@PublicEvolving public ExecutionConfig setLatencyTrackingInterval(long interval)
Setting a tracking interval <= 0 disables the latency tracking.
interval
- Interval in milliseconds.@PublicEvolving public long getLatencyTrackingInterval()
@Internal public boolean isLatencyTrackingConfigured()
public int getParallelism()
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).
PARALLELISM_DEFAULT
if the environment's default parallelism should be
used.public ExecutionConfig setParallelism(int parallelism)
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.
parallelism
- The parallelism to use@PublicEvolving public int getMaxParallelism()
The maximum degree of parallelism specifies the upper limit for dynamic scaling. It also defines the number of key groups used for partitioned state.
@PublicEvolving public void setMaxParallelism(int maxParallelism)
The maximum degree of parallelism specifies the upper limit for dynamic scaling. It also defines the number of key groups used for partitioned state.
maxParallelism
- Maximum degree of parallelism to be used for the program.public long getTaskCancellationInterval()
public ExecutionConfig setTaskCancellationInterval(long interval)
interval
- the interval (in milliseconds).@PublicEvolving public long getTaskCancellationTimeout()
The value 0
means that the timeout is disabled. In this case a stuck
cancellation will not lead to a fatal error.
@PublicEvolving public ExecutionConfig setTaskCancellationTimeout(long timeout)
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.
timeout
- The task cancellation timeout (in milliseconds).@PublicEvolving public void setRestartStrategy(RestartStrategies.RestartStrategyConfiguration restartStrategyConfiguration)
ExecutionConfig config = env.getConfig();
config.setRestartStrategy(RestartStrategies.fixedDelayRestart(
10, // number of retries
1000 // delay between retries));
restartStrategyConfiguration
- Configuration defining the restart strategy to use@PublicEvolving public RestartStrategies.RestartStrategyConfiguration getRestartStrategy()
@Deprecated public int getNumberOfExecutionRetries()
-1
indicates that the system default value (as defined in the configuration) should be used.@Deprecated public long getExecutionRetryDelay()
@Deprecated public ExecutionConfig setNumberOfExecutionRetries(int numberOfExecutionRetries)
setRestartStrategy(org.apache.flink.api.common.restartstrategy.RestartStrategies.RestartStrategyConfiguration)
. The RestartStrategies.FixedDelayRestartStrategyConfiguration
contains the number of
execution retries.-1
indicates that the system default value (as
defined in the configuration) should be used.numberOfExecutionRetries
- The number of times the system will try to re-execute failed
tasks.@Deprecated public ExecutionConfig setExecutionRetryDelay(long executionRetryDelay)
setRestartStrategy(org.apache.flink.api.common.restartstrategy.RestartStrategies.RestartStrategyConfiguration)
. The RestartStrategies.FixedDelayRestartStrategyConfiguration
contains the delay between
successive execution attempts.executionRetryDelay
- The number of milliseconds the system will wait to retry.public void setExecutionMode(ExecutionMode executionMode)
The default execution mode is ExecutionMode.PIPELINED
.
executionMode
- The execution mode to use.public ExecutionMode getExecutionMode()
The default execution mode is ExecutionMode.PIPELINED
.
@PublicEvolving public void setDefaultInputDependencyConstraint(InputDependencyConstraint inputDependencyConstraint)
The default constraint is InputDependencyConstraint.ANY
.
inputDependencyConstraint
- The input dependency constraint.@PublicEvolving public InputDependencyConstraint getDefaultInputDependencyConstraint()
The default constraint is InputDependencyConstraint.ANY
.
public void enableForceKryo()
public void disableForceKryo()
public boolean isForceKryoEnabled()
public void enableGenericTypes()
Generic types are enabled by default.
disableGenericTypes()
public void disableGenericTypes()
UnsupportedOperationException
whenever it encounters a
data type that would go through Kryo for serialization.
Disabling generic types can be helpful to eagerly find and eliminate the use of types that would go through Kryo serialization during runtime. Rather than checking types individually, using this option will throw exceptions eagerly in the places where generic types are used.
Important: We recommend to use this option only during development and pre-production phases, not during actual production use. The application program and/or the input data may be such that new, previously unseen, types occur at some point. In that case, setting this option would cause the program to fail.
enableGenericTypes()
public boolean hasGenericTypesDisabled()
Generic types are enabled by default.
enableGenericTypes()
,
disableGenericTypes()
public void enableAutoGeneratedUIDs()
disableAutoGeneratedUIDs()
public void disableAutoGeneratedUIDs()
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.
public boolean hasAutoGeneratedUIDsEnabled()
Auto generated UIDs are enabled by default.
public void enableForceAvro()
Important: Make sure to include the flink-avro module.
public void disableForceAvro()
public boolean isForceAvroEnabled()
public ExecutionConfig enableObjectReuse()
public ExecutionConfig disableObjectReuse()
public boolean isObjectReuseEnabled()
public ExecutionConfig.GlobalJobParameters getGlobalJobParameters()
public void setGlobalJobParameters(ExecutionConfig.GlobalJobParameters globalJobParameters)
globalJobParameters
- Custom user configuration objectpublic <T extends com.esotericsoftware.kryo.Serializer<?> & Serializable> void addDefaultKryoSerializer(Class<?> type, T serializer)
Note that the serializer instance must be serializable (as defined by java.io.Serializable), because it may be distributed to the worker nodes by java serialization.
type
- The class of the types serialized with the given serializer.serializer
- The serializer to use.public void addDefaultKryoSerializer(Class<?> type, Class<? extends com.esotericsoftware.kryo.Serializer<?>> serializerClass)
type
- The class of the types serialized with the given serializer.serializerClass
- The class of the serializer to use.public <T extends com.esotericsoftware.kryo.Serializer<?> & Serializable> void registerTypeWithKryoSerializer(Class<?> type, T serializer)
Note that the serializer instance must be serializable (as defined by java.io.Serializable), because it may be distributed to the worker nodes by java serialization.
type
- The class of the types serialized with the given serializer.serializer
- The serializer to use.public void registerTypeWithKryoSerializer(Class<?> type, Class<? extends com.esotericsoftware.kryo.Serializer> serializerClass)
type
- The class of the types serialized with the given serializer.serializerClass
- The class of the serializer to use.public void registerPojoType(Class<?> type)
type
- The class of the type to register.public void registerKryoType(Class<?> type)
type
- The class of the type to register.public LinkedHashMap<Class<?>,ExecutionConfig.SerializableSerializer<?>> getRegisteredTypesWithKryoSerializers()
public LinkedHashMap<Class<?>,Class<? extends com.esotericsoftware.kryo.Serializer<?>>> getRegisteredTypesWithKryoSerializerClasses()
public LinkedHashMap<Class<?>,ExecutionConfig.SerializableSerializer<?>> getDefaultKryoSerializers()
public LinkedHashMap<Class<?>,Class<? extends com.esotericsoftware.kryo.Serializer<?>>> getDefaultKryoSerializerClasses()
public LinkedHashSet<Class<?>> getRegisteredKryoTypes()
public LinkedHashSet<Class<?>> getRegisteredPojoTypes()
public boolean isAutoTypeRegistrationDisabled()
public void disableAutoTypeRegistration()
public boolean isUseSnapshotCompression()
public void setUseSnapshotCompression(boolean useSnapshotCompression)
public boolean canEqual(Object obj)
@Internal public ArchivedExecutionConfig archive()
archive
in interface Archiveable<ArchivedExecutionConfig>
public void configure(ReadableConfig configuration, ClassLoader classLoader)
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.
configuration
- a configuration to read the values fromclassLoader
- a class loader to use when loading classesCopyright © 2014–2021 The Apache Software Foundation. All rights reserved.