Interface FlinkPipelineOptions

All Superinterfaces:
ApplicationNameOptions, FileStagingOptions, HasDisplayData, PipelineOptions, StreamingOptions, 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).

  • Field Details

  • Method Details

    • getFlinkMaster

      @String("[auto]") String 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

      void setFlinkMaster(String value)
    • getParallelism

      @Integer(-1) Integer getParallelism()
    • setParallelism

      void setParallelism(Integer value)
    • getMaxParallelism

      @Integer(-1) Integer getMaxParallelism()
    • setMaxParallelism

      void setMaxParallelism(Integer value)
    • getCheckpointingInterval

      @Long(-1L) Long getCheckpointingInterval()
    • setCheckpointingInterval

      void setCheckpointingInterval(Long interval)
    • getCheckpointingMode

      @String("EXACTLY_ONCE") String getCheckpointingMode()
    • setCheckpointingMode

      void setCheckpointingMode(String mode)
    • getCheckpointTimeoutMillis

      @Long(-1L) Long getCheckpointTimeoutMillis()
    • setCheckpointTimeoutMillis

      void setCheckpointTimeoutMillis(Long checkpointTimeoutMillis)
    • getMinPauseBetweenCheckpoints

      @Long(-1L) Long getMinPauseBetweenCheckpoints()
    • setMinPauseBetweenCheckpoints

      void setMinPauseBetweenCheckpoints(Long minPauseInterval)
    • getNumConcurrentCheckpoints

      @Integer(1) int getNumConcurrentCheckpoints()
    • setNumConcurrentCheckpoints

      void setNumConcurrentCheckpoints(int maxConcurrentCheckpoints)
    • getFailOnCheckpointingErrors

      @Boolean(true) Boolean getFailOnCheckpointingErrors()
    • setFailOnCheckpointingErrors

      void setFailOnCheckpointingErrors(Boolean failOnCheckpointingErrors)
    • getFinishBundleBeforeCheckpointing

      @Boolean(false) boolean getFinishBundleBeforeCheckpointing()
    • setFinishBundleBeforeCheckpointing

      void setFinishBundleBeforeCheckpointing(boolean finishBundleBeforeCheckpointing)
    • getUnalignedCheckpointEnabled

      @Boolean(false) boolean getUnalignedCheckpointEnabled()
    • setUnalignedCheckpointEnabled

      void setUnalignedCheckpointEnabled(boolean unalignedCheckpointEnabled)
    • getForceUnalignedCheckpointEnabled

      @Boolean(false) boolean getForceUnalignedCheckpointEnabled()
    • setForceUnalignedCheckpointEnabled

      void setForceUnalignedCheckpointEnabled(boolean forceUnalignedCheckpointEnabled)
    • getShutdownSourcesAfterIdleMs

      @Long(-1L) Long getShutdownSourcesAfterIdleMs()
    • setShutdownSourcesAfterIdleMs

      void setShutdownSourcesAfterIdleMs(Long timeoutMs)
    • getNumberOfExecutionRetries

      @Integer(-1) Integer getNumberOfExecutionRetries()
    • setNumberOfExecutionRetries

      void setNumberOfExecutionRetries(Integer retries)
    • getJobCheckIntervalInSecs

      @Integer(5) int getJobCheckIntervalInSecs()
    • setJobCheckIntervalInSecs

      void setJobCheckIntervalInSecs(int seconds)
    • getAttachedMode

      @Boolean(true) boolean getAttachedMode()
    • setAttachedMode

      void setAttachedMode(boolean attachedMode)
    • getExecutionRetryDelay

      @Long(-1L) Long getExecutionRetryDelay()
    • setExecutionRetryDelay

      void setExecutionRetryDelay(Long delay)
    • getObjectReuse

      @Boolean(false) Boolean getObjectReuse()
    • setObjectReuse

      void setObjectReuse(Boolean reuse)
    • getOperatorChaining

      @Boolean(true) Boolean getOperatorChaining()
    • setOperatorChaining

      void setOperatorChaining(Boolean chaining)
    • getStateBackendFactory

      @Deprecated Class<? extends FlinkStateBackendFactory> 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

      void setStateBackend(String stateBackend)
    • getStateBackend

      String getStateBackend()
    • setStateBackendStoragePath

      void setStateBackendStoragePath(String path)
    • getStateBackendStoragePath

      String getStateBackendStoragePath()
    • getDisableMetrics

      @Boolean(false) Boolean getDisableMetrics()
    • setDisableMetrics

      void setDisableMetrics(Boolean disableMetrics)
    • isExternalizedCheckpointsEnabled

      @Boolean(false) Boolean isExternalizedCheckpointsEnabled()
      Enables or disables externalized checkpoints.
    • setExternalizedCheckpointsEnabled

      void setExternalizedCheckpointsEnabled(Boolean externalCheckpoints)
    • getRetainExternalizedCheckpointsOnCancellation

      @Boolean(false) Boolean getRetainExternalizedCheckpointsOnCancellation()
    • setRetainExternalizedCheckpointsOnCancellation

      void setRetainExternalizedCheckpointsOnCancellation(Boolean retainOnCancellation)
    • getMaxBundleSize

    • setMaxBundleSize

      void setMaxBundleSize(Long size)
    • getMaxBundleTimeMills

    • setMaxBundleTimeMills

      void setMaxBundleTimeMills(Long time)
    • getLatencyTrackingInterval

      @Long(0L) Long getLatencyTrackingInterval()
    • setLatencyTrackingInterval

      void setLatencyTrackingInterval(Long interval)
    • getAutoWatermarkInterval

      Long getAutoWatermarkInterval()
    • setAutoWatermarkInterval

      void setAutoWatermarkInterval(Long interval)
    • getExecutionModeForBatch

      @String("PIPELINED") String getExecutionModeForBatch()
    • setExecutionModeForBatch

      void setExecutionModeForBatch(String executionMode)
    • getSavepointPath

      String getSavepointPath()
    • setSavepointPath

      void setSavepointPath(String path)
    • getAllowNonRestoredState

      @Boolean(false) Boolean getAllowNonRestoredState()
    • setAllowNonRestoredState

      void setAllowNonRestoredState(Boolean allowNonRestoredState)
    • isAutoBalanceWriteFilesShardingEnabled

      @Boolean(false) Boolean isAutoBalanceWriteFilesShardingEnabled()
    • setAutoBalanceWriteFilesShardingEnabled

      void setAutoBalanceWriteFilesShardingEnabled(Boolean autoBalanceWriteFilesShardingEnabled)
    • getReportCheckpointDuration

      String getReportCheckpointDuration()
    • setReportCheckpointDuration

      void setReportCheckpointDuration(String metricNamespace)
    • getReIterableGroupByKeyResult

      @Boolean(false) Boolean getReIterableGroupByKeyResult()
    • setReIterableGroupByKeyResult

      void setReIterableGroupByKeyResult(Boolean reIterableGroupByKeyResult)
    • getFasterCopy

      @Boolean(false) Boolean getFasterCopy()
    • setFasterCopy

      void setFasterCopy(Boolean fasterCopy)
    • getFlinkConfDir

      String getFlinkConfDir()
    • setFlinkConfDir

      void setFlinkConfDir(String confDir)
    • getFileInputSplitMaxSizeMB

      @Long(0L) Long getFileInputSplitMaxSizeMB()
    • setFileInputSplitMaxSizeMB

      void setFileInputSplitMaxSizeMB(Long fileInputSplitMaxSizeMB)
    • getEnableStableInputDrain

      @Boolean(false) Boolean getEnableStableInputDrain()
    • setEnableStableInputDrain

      void setEnableStableInputDrain(Boolean enableStableInputDrain)
    • getForceSlotSharingGroup

      @Boolean(true) Boolean getForceSlotSharingGroup()
    • setForceSlotSharingGroup

      void setForceSlotSharingGroup(Boolean enableStableInputDrain)
    • defaults

      static FlinkPipelineOptions defaults()