Package org.apache.beam.runners.flink
Interface FlinkPipelineOptions
- All Superinterfaces:
ApplicationNameOptions
,FileStagingOptions
,HasDisplayData
,PipelineOptions
,StreamingOptions
,VersionDependentFlinkPipelineOptions
public interface FlinkPipelineOptions
extends PipelineOptions, ApplicationNameOptions, StreamingOptions, FileStagingOptions, VersionDependentFlinkPipelineOptions
Options which can be used to configure the Flink Runner.
Avoid using `org.apache.flink.*` members below. This allows including the flink runner without requiring flink on the classpath (e.g. to use with the direct runner).
-
Nested Class Summary
Nested ClassesModifier and TypeInterfaceDescriptionstatic class
Maximum bundle size factory.static class
Maximum bundle time factory.Nested classes/interfaces inherited from interface org.apache.beam.sdk.options.PipelineOptions
PipelineOptions.AtomicLongFactory, PipelineOptions.CheckEnabled, PipelineOptions.DirectRunner, PipelineOptions.JobNameFactory, PipelineOptions.UserAgentFactory
-
Field Summary
Fields -
Method Summary
Modifier and TypeMethodDescriptionstatic FlinkPipelineOptions
defaults()
boolean
boolean
The url of the Flink JobManager on which to execute pipelines.boolean
int
int
Class
<? extends FlinkStateBackendFactory> Deprecated.Please use setStateBackend below.boolean
Enables or disables externalized checkpoints.void
setAllowNonRestoredState
(Boolean allowNonRestoredState) void
setAttachedMode
(boolean attachedMode) void
setAutoBalanceWriteFilesShardingEnabled
(Boolean autoBalanceWriteFilesShardingEnabled) void
setAutoWatermarkInterval
(Long interval) void
setCheckpointingInterval
(Long interval) void
setCheckpointingMode
(String mode) void
setCheckpointTimeoutMillis
(Long checkpointTimeoutMillis) void
setDisableMetrics
(Boolean disableMetrics) void
setEnableStableInputDrain
(Boolean enableStableInputDrain) void
setExecutionModeForBatch
(String executionMode) void
setExecutionRetryDelay
(Long delay) void
setExternalizedCheckpointsEnabled
(Boolean externalCheckpoints) void
setFailOnCheckpointingErrors
(Boolean failOnCheckpointingErrors) void
setFasterCopy
(Boolean fasterCopy) void
setFileInputSplitMaxSizeMB
(Long fileInputSplitMaxSizeMB) void
setFinishBundleBeforeCheckpointing
(boolean finishBundleBeforeCheckpointing) void
setFlinkConfDir
(String confDir) void
setFlinkMaster
(String value) void
setForceSlotSharingGroup
(Boolean enableStableInputDrain) void
setForceUnalignedCheckpointEnabled
(boolean forceUnalignedCheckpointEnabled) void
setJobCheckIntervalInSecs
(int seconds) void
setLatencyTrackingInterval
(Long interval) void
setMaxBundleSize
(Long size) void
setMaxBundleTimeMills
(Long time) void
setMaxParallelism
(Integer value) void
setMinPauseBetweenCheckpoints
(Long minPauseInterval) void
setNumberOfExecutionRetries
(Integer retries) void
setNumConcurrentCheckpoints
(int maxConcurrentCheckpoints) void
setObjectReuse
(Boolean reuse) void
setOperatorChaining
(Boolean chaining) void
setParallelism
(Integer value) void
setReIterableGroupByKeyResult
(Boolean reIterableGroupByKeyResult) void
setReportCheckpointDuration
(String metricNamespace) void
setRetainExternalizedCheckpointsOnCancellation
(Boolean retainOnCancellation) void
setSavepointPath
(String path) void
setShutdownSourcesAfterIdleMs
(Long timeoutMs) void
setStateBackend
(String stateBackend) void
setStateBackendFactory
(Class<? extends FlinkStateBackendFactory> stateBackendFactory) Deprecated.Please use setStateBackend below.void
void
setUnalignedCheckpointEnabled
(boolean unalignedCheckpointEnabled) Methods inherited from interface org.apache.beam.sdk.options.ApplicationNameOptions
getAppName, setAppName
Methods inherited from interface org.apache.beam.sdk.options.FileStagingOptions
getFilesToStage, setFilesToStage
Methods inherited from interface org.apache.beam.sdk.transforms.display.HasDisplayData
populateDisplayData
Methods inherited from interface org.apache.beam.sdk.options.PipelineOptions
as, getJobName, getOptionsId, getRunner, getStableUniqueNames, getTempLocation, getUserAgent, outputRuntimeOptions, revision, setJobName, setOptionsId, setRunner, setStableUniqueNames, setTempLocation, setUserAgent
Methods inherited from interface org.apache.beam.sdk.options.StreamingOptions
getUpdateCompatibilityVersion, isStreaming, setStreaming, setUpdateCompatibilityVersion
Methods inherited from interface org.apache.beam.runners.flink.VersionDependentFlinkPipelineOptions
getUseDataStreamForBatch, setUseDataStreamForBatch
-
Field Details
-
AUTO
- See Also:
-
PIPELINED
- See Also:
-
EXACTLY_ONCE
- See Also:
-
-
Method Details
-
getFlinkMaster
The url of the Flink JobManager on which to execute pipelines. This can either be the address of a cluster JobManager, in the form "host:port" or one of the special Strings "[local]", "[collection]" or "[auto]". "[local]" will start a local Flink Cluster in the JVM, "[collection]" will execute the pipeline on Java Collections while "[auto]" will let the system decide where to execute the pipeline based on the environment. -
setFlinkMaster
-
getParallelism
-
setParallelism
-
getMaxParallelism
-
setMaxParallelism
-
getCheckpointingInterval
-
setCheckpointingInterval
-
getCheckpointingMode
-
setCheckpointingMode
-
getCheckpointTimeoutMillis
-
setCheckpointTimeoutMillis
-
getMinPauseBetweenCheckpoints
-
setMinPauseBetweenCheckpoints
-
getNumConcurrentCheckpoints
-
setNumConcurrentCheckpoints
void setNumConcurrentCheckpoints(int maxConcurrentCheckpoints) -
getFailOnCheckpointingErrors
-
setFailOnCheckpointingErrors
-
getFinishBundleBeforeCheckpointing
-
setFinishBundleBeforeCheckpointing
void setFinishBundleBeforeCheckpointing(boolean finishBundleBeforeCheckpointing) -
getUnalignedCheckpointEnabled
-
setUnalignedCheckpointEnabled
void setUnalignedCheckpointEnabled(boolean unalignedCheckpointEnabled) -
getForceUnalignedCheckpointEnabled
-
setForceUnalignedCheckpointEnabled
void setForceUnalignedCheckpointEnabled(boolean forceUnalignedCheckpointEnabled) -
getShutdownSourcesAfterIdleMs
-
setShutdownSourcesAfterIdleMs
-
getNumberOfExecutionRetries
-
setNumberOfExecutionRetries
-
getJobCheckIntervalInSecs
-
setJobCheckIntervalInSecs
void setJobCheckIntervalInSecs(int seconds) -
getAttachedMode
-
setAttachedMode
void setAttachedMode(boolean attachedMode) -
getExecutionRetryDelay
-
setExecutionRetryDelay
-
getObjectReuse
-
setObjectReuse
-
getOperatorChaining
-
setOperatorChaining
-
getStateBackendFactory
Deprecated.Please use setStateBackend below.State backend to store Beam's state during computation. Note: Only applicable when executing in streaming mode. -
setStateBackendFactory
@Deprecated void setStateBackendFactory(Class<? extends FlinkStateBackendFactory> stateBackendFactory) Deprecated.Please use setStateBackend below. -
setStateBackend
-
getStateBackend
String getStateBackend() -
setStateBackendStoragePath
-
getStateBackendStoragePath
String getStateBackendStoragePath() -
getDisableMetrics
-
setDisableMetrics
-
isExternalizedCheckpointsEnabled
Enables or disables externalized checkpoints. -
setExternalizedCheckpointsEnabled
-
getRetainExternalizedCheckpointsOnCancellation
-
setRetainExternalizedCheckpointsOnCancellation
-
getMaxBundleSize
-
setMaxBundleSize
-
getMaxBundleTimeMills
-
setMaxBundleTimeMills
-
getLatencyTrackingInterval
-
setLatencyTrackingInterval
-
getAutoWatermarkInterval
Long getAutoWatermarkInterval() -
setAutoWatermarkInterval
-
getExecutionModeForBatch
-
setExecutionModeForBatch
-
getSavepointPath
String getSavepointPath() -
setSavepointPath
-
getAllowNonRestoredState
-
setAllowNonRestoredState
-
isAutoBalanceWriteFilesShardingEnabled
-
setAutoBalanceWriteFilesShardingEnabled
-
getReportCheckpointDuration
String getReportCheckpointDuration() -
setReportCheckpointDuration
-
getReIterableGroupByKeyResult
-
setReIterableGroupByKeyResult
-
getFasterCopy
-
setFasterCopy
-
getFlinkConfDir
String getFlinkConfDir() -
setFlinkConfDir
-
getFileInputSplitMaxSizeMB
-
setFileInputSplitMaxSizeMB
-
getEnableStableInputDrain
-
setEnableStableInputDrain
-
getForceSlotSharingGroup
-
setForceSlotSharingGroup
-
defaults
-