@PublicEvolving public class CheckpointingOptions extends Object
Modifier and Type | Field and Description |
---|---|
static ConfigOption<Duration> |
ALIGNED_CHECKPOINT_TIMEOUT |
static ConfigOption<Boolean> |
ASYNC_SNAPSHOTS
Deprecated.
Checkpoints are always asynchronous.
|
static ConfigOption<String> |
CHECKPOINT_STORAGE
The checkpoint storage used to checkpoint state for recovery.
|
static ConfigOption<CheckpointingMode> |
CHECKPOINTING_CONSISTENCY_MODE |
static ConfigOption<Duration> |
CHECKPOINTING_INTERVAL |
static ConfigOption<Duration> |
CHECKPOINTING_INTERVAL_DURING_BACKLOG |
static ConfigOption<Duration> |
CHECKPOINTING_TIMEOUT |
static ConfigOption<String> |
CHECKPOINTS_DIRECTORY
The default directory used for storing the data files and meta data of checkpoints in a Flink
supported filesystem.
|
static ConfigOption<Boolean> |
CLEANER_PARALLEL_MODE |
static ConfigOption<Boolean> |
CREATE_CHECKPOINT_SUB_DIR
Whether to create sub-directories named by job id to store the data files and meta data of
checkpoints.
|
static ConfigOption<Boolean> |
ENABLE_CHECKPOINTS_AFTER_TASKS_FINISH |
static ConfigOption<Boolean> |
ENABLE_UNALIGNED |
static ConfigOption<Boolean> |
ENABLE_UNALIGNED_INTERRUPTIBLE_TIMERS |
static ConfigOption<ExternalizedCheckpointRetention> |
EXTERNALIZED_CHECKPOINT_RETENTION |
static ConfigOption<Boolean> |
FILE_MERGING_ACROSS_BOUNDARY
Whether to allow merging data of multiple checkpoints into one physical file.
|
static ConfigOption<Boolean> |
FILE_MERGING_ENABLED
Whether to enable merging multiple checkpoint files into one, which will greatly reduce the
number of small checkpoint files.
|
static ConfigOption<MemorySize> |
FILE_MERGING_MAX_FILE_SIZE
The max size of a physical file for merged checkpoints.
|
static ConfigOption<Float> |
FILE_MERGING_MAX_SPACE_AMPLIFICATION
Space amplification stands for the magnification of the occupied space compared to the amount
of valid data.
|
static ConfigOption<Integer> |
FILE_MERGING_MAX_SUBTASKS_PER_FILE
The upper limit of the file pool size based on the number of subtasks within each TM (only
for merging private state at Task Manager level).
|
static ConfigOption<Boolean> |
FILE_MERGING_POOL_BLOCKING
Whether to use Blocking or Non-Blocking pool for merging physical files.
|
static ConfigOption<Boolean> |
FORCE_UNALIGNED |
static ConfigOption<MemorySize> |
FS_SMALL_FILE_THRESHOLD
The minimum size of state data files.
|
static ConfigOption<Integer> |
FS_WRITE_BUFFER_SIZE
The default size of the write buffer for the checkpoint streams that write to file systems.
|
static ConfigOption<Boolean> |
INCREMENTAL_CHECKPOINTS
Option whether to create incremental checkpoints, if possible.
|
static ConfigOption<Boolean> |
LOCAL_BACKUP_ENABLED
This option configures local backup for the state backend, which indicates whether to make
backup checkpoint on local disk.
|
static ConfigOption<Boolean> |
LOCAL_RECOVERY
Deprecated.
use
StateRecoveryOptions.LOCAL_RECOVERY and LOCAL_BACKUP_ENABLED instead. |
static ConfigOption<String> |
LOCAL_RECOVERY_TASK_MANAGER_STATE_ROOT_DIRS
The config parameter defining the root directories for storing file-based state for local
recovery.
|
static ConfigOption<Integer> |
MAX_CONCURRENT_CHECKPOINTS |
static ConfigOption<Integer> |
MAX_RETAINED_CHECKPOINTS
The maximum number of completed checkpoints to retain.
|
static ConfigOption<Duration> |
MIN_PAUSE_BETWEEN_CHECKPOINTS |
static ConfigOption<String> |
SAVEPOINT_DIRECTORY
The default directory for savepoints.
|
static ConfigOption<String> |
STATE_BACKEND
Deprecated.
|
static ConfigOption<Integer> |
TOLERABLE_FAILURE_NUMBER |
static ConfigOption<Integer> |
UNALIGNED_MAX_SUBTASKS_PER_CHANNEL_STATE_FILE |
Constructor and Description |
---|
CheckpointingOptions() |
@Deprecated public static final ConfigOption<String> STATE_BACKEND
StateBackendOptions.STATE_BACKEND
.The implementation can be specified either via their shortcut name, or via the class name
of a StateBackendFactory
. If a StateBackendFactory class name is specified, the
factory is instantiated (via its zero-argument constructor) and its StateBackendFactory#createFromConfig(ReadableConfig, ClassLoader)
method is called.
Recognized shortcut names are 'hashmap' and 'rocksdb'.
public static final ConfigOption<String> CHECKPOINT_STORAGE
The implementation can be specified either via their shortcut name, or via the class name
of a CheckpointStorageFactory
. If a CheckpointStorageFactory class name is specified,
the factory is instantiated (via its zero-argument constructor) and its CheckpointStorageFactory#createFromConfig(ReadableConfig, ClassLoader)
method is called.
Recognized shortcut names are 'jobmanager' and 'filesystem'.
CHECKPOINT_STORAGE
and CHECKPOINTS_DIRECTORY
are usually combined to
configure the checkpoint location. By default, the checkpoint meta data and actual program
state will be stored in the JobManager's memory directly.
CHECKPOINT_STORAGE
is set to 'jobmanager', if CHECKPOINTS_DIRECTORY
is configured, the meta data of checkpoints will be persisted to
the path specified by CHECKPOINTS_DIRECTORY
. Otherwise, the meta data will be
stored in the JobManager's memory.
CHECKPOINT_STORAGE
is set to 'filesystem', a valid path must be configured
to CHECKPOINTS_DIRECTORY
, and the checkpoint meta data and actual program state
will both be persisted to the path.public static final ConfigOption<Integer> MAX_RETAINED_CHECKPOINTS
public static final ConfigOption<Boolean> CLEANER_PARALLEL_MODE
@Deprecated public static final ConfigOption<Boolean> ASYNC_SNAPSHOTS
public static final ConfigOption<Boolean> INCREMENTAL_CHECKPOINTS
Once enabled, the state size shown in web UI or fetched from rest API only represents the delta checkpoint size instead of full checkpoint size.
Some state backends may not support incremental checkpoints and ignore this option.
@Deprecated public static final ConfigOption<Boolean> LOCAL_RECOVERY
StateRecoveryOptions.LOCAL_RECOVERY
and LOCAL_BACKUP_ENABLED
instead.Local recovery currently only covers keyed state backends (including both the EmbeddedRocksDBStateBackend and the HashMapStateBackend).
public static final ConfigOption<String> LOCAL_RECOVERY_TASK_MANAGER_STATE_ROOT_DIRS
Local recovery currently only covers keyed state backends. Currently, MemoryStateBackend does not support local recovery and ignore this option.
public static final ConfigOption<String> SAVEPOINT_DIRECTORY
public static final ConfigOption<String> CHECKPOINTS_DIRECTORY
CHECKPOINT_STORAGE
is set
to 'jobmanager', only the meta data of checkpoints will be stored in this directory.public static final ConfigOption<Boolean> CREATE_CHECKPOINT_SUB_DIR
public static final ConfigOption<MemorySize> FS_SMALL_FILE_THRESHOLD
public static final ConfigOption<Integer> FS_WRITE_BUFFER_SIZE
public static final ConfigOption<Boolean> LOCAL_BACKUP_ENABLED
StateRecoveryOptions.LOCAL_RECOVERY
. By default, local backup is deactivated. Local backup
currently only covers keyed state backends (including both the EmbeddedRocksDBStateBackend
and the HashMapStateBackend).@Experimental public static final ConfigOption<Boolean> FILE_MERGING_ENABLED
Note: This is an experimental feature under evaluation, make sure you're aware of the possible effects of enabling it.
@Experimental public static final ConfigOption<Boolean> FILE_MERGING_ACROSS_BOUNDARY
@Experimental public static final ConfigOption<MemorySize> FILE_MERGING_MAX_FILE_SIZE
@Experimental public static final ConfigOption<Boolean> FILE_MERGING_POOL_BLOCKING
@Experimental public static final ConfigOption<Integer> FILE_MERGING_MAX_SUBTASKS_PER_FILE
TODO: remove '@Documentation.ExcludeFromDocumentation' after the feature is implemented.
@Experimental public static final ConfigOption<Float> FILE_MERGING_MAX_SPACE_AMPLIFICATION
public static final ConfigOption<CheckpointingMode> CHECKPOINTING_CONSISTENCY_MODE
public static final ConfigOption<Duration> CHECKPOINTING_TIMEOUT
public static final ConfigOption<Integer> MAX_CONCURRENT_CHECKPOINTS
public static final ConfigOption<Duration> MIN_PAUSE_BETWEEN_CHECKPOINTS
public static final ConfigOption<Integer> TOLERABLE_FAILURE_NUMBER
public static final ConfigOption<ExternalizedCheckpointRetention> EXTERNALIZED_CHECKPOINT_RETENTION
public static final ConfigOption<Duration> CHECKPOINTING_INTERVAL_DURING_BACKLOG
public static final ConfigOption<Duration> CHECKPOINTING_INTERVAL
public static final ConfigOption<Boolean> ENABLE_UNALIGNED
public static final ConfigOption<Duration> ALIGNED_CHECKPOINT_TIMEOUT
public static final ConfigOption<Boolean> FORCE_UNALIGNED
@Experimental public static final ConfigOption<Boolean> ENABLE_UNALIGNED_INTERRUPTIBLE_TIMERS
public static final ConfigOption<Boolean> ENABLE_CHECKPOINTS_AFTER_TASKS_FINISH
public static final ConfigOption<Integer> UNALIGNED_MAX_SUBTASKS_PER_CHANNEL_STATE_FILE
Copyright © 2014–2024 The Apache Software Foundation. All rights reserved.