public abstract class AbstractInvokable extends Object implements TaskInvokable, CheckpointableTask, CoordinatedTask
TaskInvokable
, CheckpointableTask
, and CoordinatedTask
with most methods throwing UnsupportedOperationException
or doing
nothing.
Any subclass that supports recoverable state and participates in checkpointing needs to
override the methods of CheckpointableTask
, such as triggerCheckpointAsync(CheckpointMetaData, CheckpointOptions)
, triggerCheckpointOnBarrier(CheckpointMetaData, CheckpointOptions, CheckpointMetricsBuilder)
,
abortCheckpointOnBarrier(long, CheckpointException)
and notifyCheckpointCompleteAsync(long)
.
Constructor and Description |
---|
AbstractInvokable(Environment environment)
Create an Invokable task and set its environment.
|
Modifier and Type | Method and Description |
---|---|
void |
abortCheckpointOnBarrier(long checkpointId,
CheckpointException cause)
Aborts a checkpoint as the result of receiving possibly some checkpoint barriers, but at
least one
CancelCheckpointMarker . |
void |
cancel()
This method is called when a task is canceled either as a result of a user abort or an
execution failure.
|
void |
cleanUp(Throwable throwable)
Cleanup any resources used in
TaskInvokable.invoke() OR TaskInvokable.restore() . |
void |
dispatchOperatorEvent(OperatorID operator,
SerializedValue<OperatorEvent> event) |
int |
getCurrentNumberOfSubtasks()
Returns the current number of subtasks the respective task is split into.
|
Environment |
getEnvironment()
Returns the environment of this task.
|
ExecutionConfig |
getExecutionConfig()
Returns the global ExecutionConfig.
|
int |
getIndexInSubtaskGroup()
Returns the index of this subtask in the subtask group.
|
Configuration |
getJobConfiguration()
Returns the job configuration object which was attached to the original
JobGraph . |
Configuration |
getTaskConfiguration()
Returns the task configuration object which was attached to the original
JobVertex . |
ClassLoader |
getUserCodeClassLoader()
Returns the user code class loader of this invokable.
|
abstract void |
invoke()
Starts the execution.
|
boolean |
isUsingNonBlockingInput() |
void |
maybeInterruptOnCancel(Thread toInterrupt,
String taskName,
Long timeout)
Checks whether the task should be interrupted during cancellation and if so, execute the
specified
Runnable interruptAction . |
Future<Void> |
notifyCheckpointAbortAsync(long checkpointId,
long latestCompletedCheckpointId)
Invoked when a checkpoint has been aborted, i.e., when the checkpoint coordinator has
received a decline message from one task and try to abort the targeted checkpoint by
notification.
|
Future<Void> |
notifyCheckpointCompleteAsync(long checkpointId)
Invoked when a checkpoint has been completed, i.e., when the checkpoint coordinator has
received the notification from all participating tasks.
|
Future<Void> |
notifyCheckpointSubsumedAsync(long checkpointId)
Invoked when a checkpoint has been subsumed, i.e., when the checkpoint coordinator has
confirmed one checkpoint has been finished, and try to remove the first previous checkpoint.
|
void |
restore()
This method can be called before
TaskInvokable.invoke() to restore an invokable object for the
last valid state, if it has it. |
CompletableFuture<Boolean> |
triggerCheckpointAsync(CheckpointMetaData checkpointMetaData,
CheckpointOptions checkpointOptions)
This method is called to trigger a checkpoint, asynchronously by the checkpoint coordinator.
|
void |
triggerCheckpointOnBarrier(CheckpointMetaData checkpointMetaData,
CheckpointOptions checkpointOptions,
CheckpointMetricsBuilder checkpointMetrics)
This method is called when a checkpoint is triggered as a result of receiving checkpoint
barriers on all input streams.
|
public AbstractInvokable(Environment environment)
environment
- The environment assigned to this invokable.public abstract void invoke() throws Exception
TaskInvokable
This method is called by the task manager when the actual execution of the task starts.
All resources should be cleaned up by calling TaskInvokable.cleanUp(Throwable)
()} after the
method returns.
invoke
in interface TaskInvokable
Exception
public void cancel() throws Exception
TaskInvokable
cancel
in interface TaskInvokable
Exception
public void cleanUp(@Nullable Throwable throwable) throws Exception
TaskInvokable
TaskInvokable.invoke()
OR TaskInvokable.restore()
. This method must be
called regardless whether the aforementioned calls succeeded or failed.cleanUp
in interface TaskInvokable
throwable
- iff failure happened during the execution of TaskInvokable.restore()
or TaskInvokable.invoke()
, null otherwise.
ATTENTION: CancelTaskException
should not be treated as a failure.
Exception
public void maybeInterruptOnCancel(Thread toInterrupt, @Nullable String taskName, @Nullable Long timeout)
TaskInvokable
Runnable interruptAction
.maybeInterruptOnCancel
in interface TaskInvokable
taskName
- optional taskName to log stack tracetimeout
- optional timeout to log stack tracepublic final Environment getEnvironment()
public final ClassLoader getUserCodeClassLoader()
public int getCurrentNumberOfSubtasks()
public int getIndexInSubtaskGroup()
public final Configuration getTaskConfiguration()
JobVertex
.JobVertex
public Configuration getJobConfiguration()
JobGraph
.JobGraph
public ExecutionConfig getExecutionConfig()
public CompletableFuture<Boolean> triggerCheckpointAsync(CheckpointMetaData checkpointMetaData, CheckpointOptions checkpointOptions)
CheckpointableTask
This method is called for tasks that start the checkpoints by injecting the initial
barriers, i.e., the source tasks. In contrast, checkpoints on downstream operators, which are
the result of receiving checkpoint barriers, invoke the CheckpointableTask.triggerCheckpointOnBarrier(CheckpointMetaData, CheckpointOptions, CheckpointMetricsBuilder)
method.
triggerCheckpointAsync
in interface CheckpointableTask
checkpointMetaData
- Meta data for about this checkpointcheckpointOptions
- Options for performing this checkpointfalse
if the checkpoint was not carried out, true
otherwisepublic void triggerCheckpointOnBarrier(CheckpointMetaData checkpointMetaData, CheckpointOptions checkpointOptions, CheckpointMetricsBuilder checkpointMetrics) throws IOException
CheckpointableTask
triggerCheckpointOnBarrier
in interface CheckpointableTask
checkpointMetaData
- Meta data for about this checkpointcheckpointOptions
- Options for performing this checkpointcheckpointMetrics
- Metrics about this checkpointIOException
- Exceptions thrown as the result of triggering a checkpoint are forwarded.public void abortCheckpointOnBarrier(long checkpointId, CheckpointException cause) throws IOException
CheckpointableTask
CancelCheckpointMarker
.
This requires implementing tasks to forward a CancelCheckpointMarker
to their outputs.
abortCheckpointOnBarrier
in interface CheckpointableTask
checkpointId
- The ID of the checkpoint to be aborted.cause
- The reason why the checkpoint was aborted during alignmentIOException
public Future<Void> notifyCheckpointCompleteAsync(long checkpointId)
CheckpointableTask
notifyCheckpointCompleteAsync
in interface CheckpointableTask
checkpointId
- The ID of the checkpoint that is complete.public Future<Void> notifyCheckpointAbortAsync(long checkpointId, long latestCompletedCheckpointId)
CheckpointableTask
notifyCheckpointAbortAsync
in interface CheckpointableTask
checkpointId
- The ID of the checkpoint that is aborted.latestCompletedCheckpointId
- The ID of the latest completed checkpoint.public Future<Void> notifyCheckpointSubsumedAsync(long checkpointId)
CheckpointableTask
notifyCheckpointSubsumedAsync
in interface CheckpointableTask
checkpointId
- The ID of the checkpoint that is subsumed.public void dispatchOperatorEvent(OperatorID operator, SerializedValue<OperatorEvent> event) throws FlinkException
dispatchOperatorEvent
in interface CoordinatedTask
FlinkException
public void restore() throws Exception
TaskInvokable
TaskInvokable.invoke()
to restore an invokable object for the
last valid state, if it has it.
If TaskInvokable.invoke()
is not called after this method for some reason (e.g. task
cancellation); then all resources should be cleaned up by calling TaskInvokable.cleanUp(Throwable)
()} after the method returns.
restore
in interface TaskInvokable
Exception
public boolean isUsingNonBlockingInput()
isUsingNonBlockingInput
in interface TaskInvokable
InputGate.getNext()
is used (as opposed to
InputGate.pollNext()
. To be removed together with the DataSet API.Copyright © 2014–2024 The Apache Software Foundation. All rights reserved.