Checkpointing

Every function and operator in Flink can be stateful (see working with state for details). Stateful functions store data across the processing of individual elements/events, making state a critical building block for any type of more elaborate operation.

In order to make state fault tolerant, Flink needs to checkpoint the state. Checkpoints allow Flink to recover state and positions in the streams to give the application the same semantics as a failure-free execution.

The documentation on streaming fault tolerance describes in detail the technique behind Flink’s streaming fault tolerance mechanism.

Prerequisites

Flink’s checkpointing mechanism interacts with durable storage for streams and state. In general, it requires:

  • A persistent (or durable) data source that can replay records for a certain amount of time. Examples for such sources are persistent messages queues (e.g., Apache Kafka, RabbitMQ, Amazon Kinesis, Google PubSub) or file systems (e.g., HDFS, S3, GFS, NFS, Ceph, …).
  • A persistent storage for state, typically a distributed filesystem (e.g., HDFS, S3, GFS, NFS, Ceph, …)

Enabling and Configuring Checkpointing

By default, checkpointing is disabled. To enable checkpointing, call enableCheckpointing(n) on the StreamExecutionEnvironment, where n is the checkpoint interval in milliseconds.

Other parameters for checkpointing include:

  • checkpoint storage: You can set the location where checkpoint snapshots are made durable. By default Flink will use the JobManager’s heap. For production deployments it is recommended to instead use a durable filesystem. See checkpoint storage for more details on the available options for job-wide and cluster-wide configuration.

  • exactly-once vs. at-least-once: You can optionally pass a mode to the enableCheckpointing(n) method to choose between the two guarantee levels. Exactly-once is preferable for most applications. At-least-once may be relevant for certain super-low-latency (consistently few milliseconds) applications.

  • checkpoint timeout: The time after which a checkpoint-in-progress is aborted, if it did not complete by then.

  • minimum time between checkpoints: To make sure that the streaming application makes a certain amount of progress between checkpoints, one can define how much time needs to pass between checkpoints. If this value is set for example to 5000, the next checkpoint will be started no sooner than 5 seconds after the previous checkpoint completed, regardless of the checkpoint duration and the checkpoint interval. Note that this implies that the checkpoint interval will never be smaller than this parameter.

    It is often easier to configure applications by defining the “time between checkpoints” than the checkpoint interval, because the “time between checkpoints” is not susceptible to the fact that checkpoints may sometimes take longer than on average (for example if the target storage system is temporarily slow).

    Note that this value also implies that the number of concurrent checkpoints is one.

  • tolerable checkpoint failure number: This defines how many consecutive checkpoint failures will be tolerated, before the whole job is failed over. The default value is 0, which means no checkpoint failures will be tolerated, and the job will fail on first reported checkpoint failure.

  • number of concurrent checkpoints: By default, the system will not trigger another checkpoint while one is still in progress. This ensures that the topology does not spend too much time on checkpoints and not make progress with processing the streams. It is possible to allow for multiple overlapping checkpoints, which is interesting for pipelines that have a certain processing delay (for example because the functions call external services that need some time to respond) but that still want to do very frequent checkpoints (100s of milliseconds) to re-process very little upon failures.

    This option cannot be used when a minimum time between checkpoints is defined.

  • externalized checkpoints: You can configure periodic checkpoints to be persisted externally. Externalized checkpoints write their meta data out to persistent storage and are not automatically cleaned up when the job fails. This way, you will have a checkpoint around to resume from if your job fails. There are more details in the deployment notes on externalized checkpoints.

  • unaligned checkpoints: You can enable unaligned checkpoints to greatly reduce checkpointing times under backpressure. This only works for exactly-once checkpoints and with one concurrent checkpoint.

  • checkpoints with finished tasks: By default Flink will continue performing checkpoints even if parts of the DAG have finished processing all of their records. Please refer to important considerations for details.

    Java

  1. StreamExecutionEnvironment env = StreamExecutionEnvironment.getExecutionEnvironment();
  2. // start a checkpoint every 1000 ms
  3. env.enableCheckpointing(1000);
  4. // advanced options:
  5. // set mode to exactly-once (this is the default)
  6. env.getCheckpointConfig().setCheckpointingMode(CheckpointingMode.EXACTLY_ONCE);
  7. // make sure 500 ms of progress happen between checkpoints
  8. env.getCheckpointConfig().setMinPauseBetweenCheckpoints(500);
  9. // checkpoints have to complete within one minute, or are discarded
  10. env.getCheckpointConfig().setCheckpointTimeout(60000);
  11. // only two consecutive checkpoint failures are tolerated
  12. env.getCheckpointConfig().setTolerableCheckpointFailureNumber(2);
  13. // allow only one checkpoint to be in progress at the same time
  14. env.getCheckpointConfig().setMaxConcurrentCheckpoints(1);
  15. // enable externalized checkpoints which are retained
  16. // after job cancellation
  17. env.getCheckpointConfig().setExternalizedCheckpointCleanup(
  18. ExternalizedCheckpointCleanup.RETAIN_ON_CANCELLATION);
  19. // enables the unaligned checkpoints
  20. env.getCheckpointConfig().enableUnalignedCheckpoints();
  21. // sets the checkpoint storage where checkpoint snapshots will be written
  22. env.getCheckpointConfig().setCheckpointStorage("hdfs:///my/checkpoint/dir");
  23. // enable checkpointing with finished tasks
  24. Configuration config = new Configuration();
  25. config.set(ExecutionCheckpointingOptions.ENABLE_CHECKPOINTS_AFTER_TASKS_FINISH, true);
  26. env.configure(config);

Scala

  1. val env = StreamExecutionEnvironment.getExecutionEnvironment()
  2. // start a checkpoint every 1000 ms
  3. env.enableCheckpointing(1000)
  4. // advanced options:
  5. // set mode to exactly-once (this is the default)
  6. env.getCheckpointConfig.setCheckpointingMode(CheckpointingMode.EXACTLY_ONCE)
  7. // make sure 500 ms of progress happen between checkpoints
  8. env.getCheckpointConfig.setMinPauseBetweenCheckpoints(500)
  9. // checkpoints have to complete within one minute, or are discarded
  10. env.getCheckpointConfig.setCheckpointTimeout(60000)
  11. // only two consecutive checkpoint failures are tolerated
  12. env.getCheckpointConfig().setTolerableCheckpointFailureNumber(2)
  13. // allow only one checkpoint to be in progress at the same time
  14. env.getCheckpointConfig.setMaxConcurrentCheckpoints(1)
  15. // enable externalized checkpoints which are retained
  16. // after job cancellation
  17. env.getCheckpointConfig().setExternalizedCheckpointCleanup(
  18. ExternalizedCheckpointCleanup.RETAIN_ON_CANCELLATION)
  19. // enables the unaligned checkpoints
  20. env.getCheckpointConfig.enableUnalignedCheckpoints()
  21. // sets the checkpoint storage where checkpoint snapshots will be written
  22. env.getCheckpointConfig.setCheckpointStorage("hdfs:///my/checkpoint/dir")
  23. // enable checkpointing with finished tasks
  24. val config = new Configuration()
  25. config.set(ExecutionCheckpointingOptions.ENABLE_CHECKPOINTS_AFTER_TASKS_FINISH, true)
  26. env.configure(config)

Python

  1. env = StreamExecutionEnvironment.get_execution_environment()
  2. # start a checkpoint every 1000 ms
  3. env.enable_checkpointing(1000)
  4. # advanced options:
  5. # set mode to exactly-once (this is the default)
  6. env.get_checkpoint_config().set_checkpointing_mode(CheckpointingMode.EXACTLY_ONCE)
  7. # make sure 500 ms of progress happen between checkpoints
  8. env.get_checkpoint_config().set_min_pause_between_checkpoints(500)
  9. # checkpoints have to complete within one minute, or are discarded
  10. env.get_checkpoint_config().set_checkpoint_timeout(60000)
  11. # only two consecutive checkpoint failures are tolerated
  12. env.get_checkpoint_config().set_tolerable_checkpoint_failure_number(2)
  13. # allow only one checkpoint to be in progress at the same time
  14. env.get_checkpoint_config().set_max_concurrent_checkpoints(1)
  15. # enable externalized checkpoints which are retained after job cancellation
  16. env.get_checkpoint_config().enable_externalized_checkpoints(ExternalizedCheckpointCleanup.RETAIN_ON_CANCELLATION)
  17. # enables the unaligned checkpoints
  18. env.get_checkpoint_config().enable_unaligned_checkpoints()

Some more parameters and/or defaults may be set via conf/flink-conf.yaml (see configuration for a full guide):

KeyDefaultTypeDescription
state.backend.incremental
falseBooleanOption whether the state backend should create incremental checkpoints, if possible. For an incremental checkpoint, only a diff from the previous checkpoint is stored, rather than the complete checkpoint state. 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.
state.backend.local-recovery
falseBooleanThis option configures local recovery for this state backend. By default, local recovery is deactivated. Local recovery currently only covers keyed state backends. Currently, the MemoryStateBackend does not support local recovery and ignores this option.
state.checkpoint-storage
(none)StringThe checkpoint storage implementation to be used to checkpoint state.
The implementation can be specified either via their shortcut name, or via the class name of a CheckpointStorageFactory. If a factory is specified it is instantiated via its zero argument constructor and its CheckpointStorageFactory#createFromConfig(ReadableConfig, ClassLoader) method is called.
Recognized shortcut names are ‘jobmanager’ and ‘filesystem’.
state.checkpoints.dir
(none)StringThe default directory used for storing the data files and meta data of checkpoints in a Flink supported filesystem. The storage path must be accessible from all participating processes/nodes(i.e. all TaskManagers and JobManagers).
state.checkpoints.num-retained
1IntegerThe maximum number of completed checkpoints to retain.
state.savepoints.dir
(none)StringThe default directory for savepoints. Used by the state backends that write savepoints to file systems (HashMapStateBackend, EmbeddedRocksDBStateBackend).
state.storage.fs.memory-threshold
20 kbMemorySizeThe minimum size of state data files. All state chunks smaller than that are stored inline in the root checkpoint metadata file. The max memory threshold for this configuration is 1MB.
state.storage.fs.write-buffer-size
4096IntegerThe default size of the write buffer for the checkpoint streams that write to file systems. The actual write buffer size is determined to be the maximum of the value of this option and option ‘state.storage.fs.memory-threshold’.
taskmanager.state.local.root-dirs
(none)StringThe config parameter defining the root directories for storing file-based state for local recovery. Local recovery currently only covers keyed state backends. Currently, MemoryStateBackend does not support local recovery and ignores this option. If not configured it will default to <WORKING_DIR>/localState. The <WORKING_DIR> can be configured via process.taskmanager.working-dir

Back to top

Selecting Checkpoint Storage

Flink’s checkpointing mechanism stores consistent snapshots of all the state in timers and stateful operators, including connectors, windows, and any user-defined state. Where the checkpoints are stored (e.g., JobManager memory, file system, database) depends on the configured Checkpoint Storage.

By default, checkpoints are stored in memory in the JobManager. For proper persistence of large state, Flink supports various approaches for checkpointing state in other locations. The choice of checkpoint storage can be configured via StreamExecutionEnvironment.getCheckpointConfig().setCheckpointStorage(…). It is strongly encouraged that checkpoints be stored in a highly-available filesystem for production deployments.

See checkpoint storage for more details on the available options for job-wide and cluster-wide configuration.

State Checkpoints in Iterative Jobs

Flink currently only provides processing guarantees for jobs without iterations. Enabling checkpointing on an iterative job causes an exception. In order to force checkpointing on an iterative program the user needs to set a special flag when enabling checkpointing: env.enableCheckpointing(interval, CheckpointingMode.EXACTLY_ONCE, force = true).

Please note that records in flight in the loop edges (and the state changes associated with them) will be lost during failure.

Checkpointing with parts of the graph finished

Starting from Flink 1.14 it is possible to continue performing checkpoints even if parts of the job graph have finished processing all data, which might happen if it contains bounded sources. This feature is enabled by default since 1.15, and it could be disabled via a feature flag:

  1. Configuration config = new Configuration();
  2. config.set(ExecutionCheckpointingOptions.ENABLE_CHECKPOINTS_AFTER_TASKS_FINISH, false);
  3. StreamExecutionEnvironment env = StreamExecutionEnvironment.getExecutionEnvironment(config);

Once the tasks/subtasks are finished, they do not contribute to the checkpoints any longer. This is an important consideration when implementing any custom operators or UDFs (User-Defined Functions).

In order to support checkpointing with tasks that finished, we adjusted the task lifecycle and introduced the StreamOperator#finish method. This method is expected to be a clear cutoff point for flushing any remaining buffered state. All checkpoints taken after the finish method has been called should be empty (in most cases) and should not contain any buffered data since there will be no way to emit this data. One notable exception is if your operator has some pointers to transactions in external systems (i.e. order to implement the exactly-once semantic). In such a case, checkpoints taken after invoking the finish() method should keep a pointer to the last transaction(s) that will be committed in the final checkpoint before the operator is closed. A good built-in example of this are exactly-once sinks and the TwoPhaseCommitSinkFunction.

How does this impact the operator state?

There is a special handling for UnionListState, which has often been used to implement a global view over offsets in an external system (i.e. storing current offsets of Kafka partitions). If we had discarded a state for a single subtask that had its close method called, we would have lost the offsets for partitions that it had been assigned. In order to work around this problem, we let checkpoints succeed only if none or all subtasks that use UnionListState are finished.

We have not seen ListState used in a similar way, but you should be aware that any state checkpointed after the close method will be discarded and not be available after a restore.

Any operator that is prepared to be rescaled should work well with tasks that partially finish. Restoring from a checkpoint where only a subset of tasks finished is equivalent to restoring such a task with the number of new subtasks equal to the number of running tasks.

Waiting for the final checkpoint before task exit

To ensure all the records could be committed for operators using the two-phase commit, the tasks would wait for the final checkpoint completed successfully after all the operators finished. It needs to be noted that this behavior would prolong the execution time of tasks. If the checkpoint interval is long, the execution time would also be prolonged largely. For the worst case, if the checkpoint interval is set to Long.MAX_VALUE, the tasks would in fact be blocked forever since the final checkpoint would never happen.

Back to top