Class AbstractInvokable
- java.lang.Object
-
- org.apache.flink.runtime.jobgraph.tasks.AbstractInvokable
-
- All Implemented Interfaces:
CheckpointableTask
,CoordinatedTask
,TaskInvokable
- Direct Known Subclasses:
BatchTask
,DataSinkTask
,DataSourceTask
,IterationSynchronizationSinkTask
public abstract class AbstractInvokable extends Object implements TaskInvokable, CheckpointableTask, CoordinatedTask
A base implementation ofTaskInvokable
,CheckpointableTask
, andCoordinatedTask
with most methods throwingUnsupportedOperationException
or doing nothing.Any subclass that supports recoverable state and participates in checkpointing needs to override the methods of
CheckpointableTask
, such astriggerCheckpointAsync(CheckpointMetaData, CheckpointOptions)
,triggerCheckpointOnBarrier(CheckpointMetaData, CheckpointOptions, CheckpointMetricsBuilder)
,abortCheckpointOnBarrier(long, CheckpointException)
andnotifyCheckpointCompleteAsync(long)
.
-
-
Constructor Summary
Constructors Constructor Description AbstractInvokable(Environment environment)
Create an Invokable task and set its environment.
-
Method Summary
All Methods Instance Methods Abstract Methods Concrete Methods Modifier and Type Method Description void
abortCheckpointOnBarrier(long checkpointId, CheckpointException cause)
Aborts a checkpoint as the result of receiving possibly some checkpoint barriers, but at least oneCancelCheckpointMarker
.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 inTaskInvokable.invoke()
ORTaskInvokable.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 originalJobGraph
.Configuration
getTaskConfiguration()
Returns the task configuration object which was attached to the originalJobVertex
.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 specifiedRunnable 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 beforeTaskInvokable.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.
-
-
-
Constructor Detail
-
AbstractInvokable
public AbstractInvokable(Environment environment)
Create an Invokable task and set its environment.- Parameters:
environment
- The environment assigned to this invokable.
-
-
Method Detail
-
invoke
public abstract void invoke() throws Exception
Description copied from interface:TaskInvokable
Starts the execution.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.- Specified by:
invoke
in interfaceTaskInvokable
- Throws:
Exception
-
cancel
public void cancel() throws Exception
Description copied from interface:TaskInvokable
This method is called when a task is canceled either as a result of a user abort or an execution failure. It can be overwritten to respond to shut down the user code properly.- Specified by:
cancel
in interfaceTaskInvokable
- Throws:
Exception
-
cleanUp
public void cleanUp(@Nullable Throwable throwable) throws Exception
Description copied from interface:TaskInvokable
Cleanup any resources used inTaskInvokable.invoke()
ORTaskInvokable.restore()
. This method must be called regardless whether the aforementioned calls succeeded or failed.- Specified by:
cleanUp
in interfaceTaskInvokable
- Parameters:
throwable
- iff failure happened during the execution ofTaskInvokable.restore()
orTaskInvokable.invoke()
, null otherwise.ATTENTION:
CancelTaskException
should not be treated as a failure.- Throws:
Exception
-
maybeInterruptOnCancel
public void maybeInterruptOnCancel(Thread toInterrupt, @Nullable String taskName, @Nullable Long timeout)
Description copied from interface:TaskInvokable
Checks whether the task should be interrupted during cancellation and if so, execute the specifiedRunnable interruptAction
.- Specified by:
maybeInterruptOnCancel
in interfaceTaskInvokable
taskName
- optional taskName to log stack tracetimeout
- optional timeout to log stack trace
-
getEnvironment
public final Environment getEnvironment()
Returns the environment of this task.- Returns:
- The environment of this task.
-
getUserCodeClassLoader
public final ClassLoader getUserCodeClassLoader()
Returns the user code class loader of this invokable.- Returns:
- user code class loader of this invokable.
-
getCurrentNumberOfSubtasks
public int getCurrentNumberOfSubtasks()
Returns the current number of subtasks the respective task is split into.- Returns:
- the current number of subtasks the respective task is split into
-
getIndexInSubtaskGroup
public int getIndexInSubtaskGroup()
Returns the index of this subtask in the subtask group.- Returns:
- the index of this subtask in the subtask group
-
getTaskConfiguration
public final Configuration getTaskConfiguration()
Returns the task configuration object which was attached to the originalJobVertex
.- Returns:
- the task configuration object which was attached to the original
JobVertex
-
getJobConfiguration
public Configuration getJobConfiguration()
Returns the job configuration object which was attached to the originalJobGraph
.- Returns:
- the job configuration object which was attached to the original
JobGraph
-
getExecutionConfig
public ExecutionConfig getExecutionConfig()
Returns the global ExecutionConfig.
-
triggerCheckpointAsync
public CompletableFuture<Boolean> triggerCheckpointAsync(CheckpointMetaData checkpointMetaData, CheckpointOptions checkpointOptions)
Description copied from interface:CheckpointableTask
This method is called to trigger a checkpoint, asynchronously by the checkpoint coordinator.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.- Specified by:
triggerCheckpointAsync
in interfaceCheckpointableTask
- Parameters:
checkpointMetaData
- Meta data for about this checkpointcheckpointOptions
- Options for performing this checkpoint- Returns:
- future with value of
false
if the checkpoint was not carried out,true
otherwise
-
triggerCheckpointOnBarrier
public void triggerCheckpointOnBarrier(CheckpointMetaData checkpointMetaData, CheckpointOptions checkpointOptions, CheckpointMetricsBuilder checkpointMetrics) throws IOException
Description copied from interface:CheckpointableTask
This method is called when a checkpoint is triggered as a result of receiving checkpoint barriers on all input streams.- Specified by:
triggerCheckpointOnBarrier
in interfaceCheckpointableTask
- Parameters:
checkpointMetaData
- Meta data for about this checkpointcheckpointOptions
- Options for performing this checkpointcheckpointMetrics
- Metrics about this checkpoint- Throws:
IOException
- Exceptions thrown as the result of triggering a checkpoint are forwarded.
-
abortCheckpointOnBarrier
public void abortCheckpointOnBarrier(long checkpointId, CheckpointException cause) throws IOException
Description copied from interface:CheckpointableTask
Aborts a checkpoint as the result of receiving possibly some checkpoint barriers, but at least oneCancelCheckpointMarker
.This requires implementing tasks to forward a
CancelCheckpointMarker
to their outputs.- Specified by:
abortCheckpointOnBarrier
in interfaceCheckpointableTask
- Parameters:
checkpointId
- The ID of the checkpoint to be aborted.cause
- The reason why the checkpoint was aborted during alignment- Throws:
IOException
-
notifyCheckpointCompleteAsync
public Future<Void> notifyCheckpointCompleteAsync(long checkpointId)
Description copied from interface:CheckpointableTask
Invoked when a checkpoint has been completed, i.e., when the checkpoint coordinator has received the notification from all participating tasks.- Specified by:
notifyCheckpointCompleteAsync
in interfaceCheckpointableTask
- Parameters:
checkpointId
- The ID of the checkpoint that is complete.- Returns:
- future that completes when the notification has been processed by the task.
-
notifyCheckpointAbortAsync
public Future<Void> notifyCheckpointAbortAsync(long checkpointId, long latestCompletedCheckpointId)
Description copied from interface:CheckpointableTask
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.- Specified by:
notifyCheckpointAbortAsync
in interfaceCheckpointableTask
- Parameters:
checkpointId
- The ID of the checkpoint that is aborted.latestCompletedCheckpointId
- The ID of the latest completed checkpoint.- Returns:
- future that completes when the notification has been processed by the task.
-
notifyCheckpointSubsumedAsync
public Future<Void> notifyCheckpointSubsumedAsync(long checkpointId)
Description copied from interface:CheckpointableTask
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.- Specified by:
notifyCheckpointSubsumedAsync
in interfaceCheckpointableTask
- Parameters:
checkpointId
- The ID of the checkpoint that is subsumed.- Returns:
- future that completes when the notification has been processed by the task.
-
dispatchOperatorEvent
public void dispatchOperatorEvent(OperatorID operator, SerializedValue<OperatorEvent> event) throws FlinkException
- Specified by:
dispatchOperatorEvent
in interfaceCoordinatedTask
- Throws:
FlinkException
-
restore
public void restore() throws Exception
Description copied from interface:TaskInvokable
This method can be called beforeTaskInvokable.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 callingTaskInvokable.cleanUp(Throwable)
()} after the method returns.- Specified by:
restore
in interfaceTaskInvokable
- Throws:
Exception
-
isUsingNonBlockingInput
public boolean isUsingNonBlockingInput()
- Specified by:
isUsingNonBlockingInput
in interfaceTaskInvokable
- Returns:
- true if blocking input such as
InputGate.getNext()
is used (as opposed toInputGate.pollNext()
. To be removed together with the DataSet API.
-
-