klion26 commented on a change in pull request #9240: [FLINK-13440] Add test
that fails job when in-flight sync savepoint is discarded.
URL: https://github.com/apache/flink/pull/9240#discussion_r315125443
##########
File path:
flink-runtime/src/main/java/org/apache/flink/runtime/checkpoint/CheckpointFailureReason.java
##########
@@ -23,60 +23,69 @@
*/
public enum CheckpointFailureReason {
- PERIODIC_SCHEDULER_SHUTDOWN("Periodic checkpoint scheduler is shut
down."),
+ PERIODIC_SCHEDULER_SHUTDOWN(true, "Periodic checkpoint scheduler is
shut down."),
- ALREADY_QUEUED("Another checkpoint request has already been queued."),
+ ALREADY_QUEUED(true, "Another checkpoint request has already been
queued."),
- TOO_MANY_CONCURRENT_CHECKPOINTS("The maximum number of concurrent
checkpoints is exceeded"),
+ TOO_MANY_CONCURRENT_CHECKPOINTS(true, "The maximum number of concurrent
checkpoints is exceeded"),
- MINIMUM_TIME_BETWEEN_CHECKPOINTS("The minimum time between checkpoints
is still pending. " +
+ MINIMUM_TIME_BETWEEN_CHECKPOINTS(true, "The minimum time between
checkpoints is still pending. " +
"Checkpoint will be triggered after the minimum time."),
- NOT_ALL_REQUIRED_TASKS_RUNNING("Not all required tasks are currently
running."),
+ NOT_ALL_REQUIRED_TASKS_RUNNING(true, "Not all required tasks are
currently running."),
- EXCEPTION("An Exception occurred while triggering the checkpoint."),
+ EXCEPTION(true, "An Exception occurred while triggering the
checkpoint."),
- CHECKPOINT_EXPIRED("Checkpoint expired before completing."),
+ CHECKPOINT_EXPIRED(false, "Checkpoint expired before completing."),
- CHECKPOINT_SUBSUMED("Checkpoint has been subsumed."),
+ CHECKPOINT_SUBSUMED(false, "Checkpoint has been subsumed."),
- CHECKPOINT_DECLINED("Checkpoint was declined."),
+ CHECKPOINT_DECLINED(false, "Checkpoint was declined."),
- CHECKPOINT_DECLINED_TASK_NOT_READY("Checkpoint was declined (tasks not
ready)"),
+ CHECKPOINT_DECLINED_TASK_NOT_READY(false, "Checkpoint was declined
(tasks not ready)"),
- CHECKPOINT_DECLINED_TASK_NOT_CHECKPOINTING("Task does not support
checkpointing"),
+ CHECKPOINT_DECLINED_TASK_NOT_CHECKPOINTING(false, "Task does not
support checkpointing"),
- CHECKPOINT_DECLINED_SUBSUMED("Checkpoint was canceled because a barrier
from newer checkpoint was received."),
+ CHECKPOINT_DECLINED_SUBSUMED(false, "Checkpoint was canceled because a
barrier from newer checkpoint was received."),
- CHECKPOINT_DECLINED_ON_CANCELLATION_BARRIER("Task received cancellation
from one of its inputs"),
+ CHECKPOINT_DECLINED_ON_CANCELLATION_BARRIER(false, "Task received
cancellation from one of its inputs"),
- CHECKPOINT_DECLINED_ALIGNMENT_LIMIT_EXCEEDED("The checkpoint alignment
phase needed to buffer more than the configured maximum bytes"),
+ CHECKPOINT_DECLINED_ALIGNMENT_LIMIT_EXCEEDED(false, "The checkpoint
alignment phase needed to buffer more than the configured maximum bytes"),
- CHECKPOINT_DECLINED_INPUT_END_OF_STREAM("Checkpoint was declined
because one input stream is finished"),
+ CHECKPOINT_DECLINED_INPUT_END_OF_STREAM(false, "Checkpoint was declined
because one input stream is finished"),
- CHECKPOINT_COORDINATOR_SHUTDOWN("CheckpointCoordinator shutdown."),
+ CHECKPOINT_COORDINATOR_SHUTDOWN(false, "CheckpointCoordinator
shutdown."),
Review comment:
https://github.com/apache/flink/blob/94fa4ceade57172362e2d35e5aac8383f8f40a40/flink-runtime/src/main/java/org/apache/flink/runtime/checkpoint/CheckpointCoordinator.java#L472
Why the `preFlight` status of `CHECKPOINT_COORDINATOR_SHUTDOWN` is false? In
`CheckpointCoordinator#triggerCheckpoint` we'll throw
`CHECKPOINT_COORDINATOR_SHUTDOWN` exception, seems the checkpoint has not
passed into task already
----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.
For queries about this service, please contact Infrastructure at:
[email protected]
With regards,
Apache Git Services