Class StreamTask<OUT,OP extends StreamOperator<OUT>>
- java.lang.Object
-
- org.apache.flink.streaming.runtime.tasks.StreamTask<OUT,OP>
-
- Type Parameters:
OUT
-OP
-
- All Implemented Interfaces:
CheckpointableTask
,CoordinatedTask
,TaskInvokable
,AsyncExceptionHandler
,ContainingTaskDetails
- Direct Known Subclasses:
AbstractTwoInputStreamTask
,MultipleInputStreamTask
,OneInputStreamTask
,SourceOperatorStreamTask
,SourceStreamTask
@Internal public abstract class StreamTask<OUT,OP extends StreamOperator<OUT>> extends Object implements TaskInvokable, CheckpointableTask, CoordinatedTask, AsyncExceptionHandler, ContainingTaskDetails
Base class for all streaming tasks. A task is the unit of local processing that is deployed and executed by the TaskManagers. Each task runs one or moreStreamOperator
s which form the Task's operator chain. Operators that are chained together execute synchronously in the same thread and hence on the same stream partition. A common case for these chains are successive map/flatmap/filter tasks.The task chain contains one "head" operator and multiple chained operators. The StreamTask is specialized for the type of the head operator: one-input and two-input tasks, as well as for sources, iteration heads and iteration tails.
The Task class deals with the setup of the streams read by the head operator, and the streams produced by the operators at the ends of the operator chain. Note that the chain may fork and thus have multiple ends.
The life cycle of the task is set up as follows:
-- setInitialState -> provides state of all operators in the chain -- invoke() | +----> Create basic utils (config, etc) and load the chain of operators +----> operators.setup() +----> task specific init() +----> initialize-operator-states() +----> open-operators() +----> run() +----> finish-operators() +----> close-operators() +----> common cleanup +----> task specific cleanup()
The
StreamTask
has a lock object calledlock
. All calls to methods on aStreamOperator
must be synchronized on this lock object to ensure that no methods are called concurrently.
-
-
Nested Class Summary
Nested Classes Modifier and Type Class Description static interface
StreamTask.CanEmitBatchOfRecordsChecker
Check whether records can be emitted in batch.
-
Field Summary
Fields Modifier and Type Field Description protected CheckpointStorage
checkpointStorage
Our checkpoint storage.protected StreamConfig
configuration
The configuration of this streaming task.protected StreamInputProcessor
inputProcessor
The input processor.protected static org.slf4j.Logger
LOG
The logger used by the StreamTask and its subclasses.protected MailboxProcessor
mailboxProcessor
protected OP
mainOperator
the main operator that consumes the input streams of this task.protected OperatorChain<OUT,OP>
operatorChain
The chain of operators executed by this task.protected RecordWriterDelegate<SerializationDelegate<StreamRecord<OUT>>>
recordWriter
protected StateBackend
stateBackend
Our state backend.protected TimerService
systemTimerService
In contrast totimerService
we should not register any user timers here.protected TimerService
timerService
The internalTimerService
used to define the current processing time (default =System.currentTimeMillis()
) and register timers for tasks to be executed in the future.static ThreadGroup
TRIGGER_THREAD_GROUP
The thread group that holds all trigger timer threads.
-
Constructor Summary
Constructors Modifier Constructor Description protected
StreamTask(Environment env)
Constructor for initialization, possibly with initial state (recovery / savepoint / etc).protected
StreamTask(Environment env, TimerService timerService)
Constructor for initialization, possibly with initial state (recovery / savepoint / etc).protected
StreamTask(Environment environment, TimerService timerService, Thread.UncaughtExceptionHandler uncaughtExceptionHandler)
protected
StreamTask(Environment environment, TimerService timerService, Thread.UncaughtExceptionHandler uncaughtExceptionHandler, StreamTaskActionExecutor actionExecutor)
Constructor for initialization, possibly with initial state (recovery / savepoint / etc).protected
StreamTask(Environment environment, TimerService timerService, Thread.UncaughtExceptionHandler uncaughtExceptionHandler, StreamTaskActionExecutor actionExecutor, TaskMailbox mailbox)
-
Method Summary
All Methods Static 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
.protected void
advanceToEndOfEventTime()
Emits theMAX_WATERMARK
so that all registered timers are fired.protected void
afterInvoke()
void
cancel()
This method is called when a task is canceled either as a result of a user abort or an execution failure.protected void
cancelTask()
void
cleanUp(Throwable throwable)
Cleanup any resources used inTaskInvokable.invoke()
ORTaskInvokable.restore()
.protected void
cleanUpInternal()
static <OUT> RecordWriterDelegate<SerializationDelegate<StreamRecord<OUT>>>
createRecordWriterDelegate(StreamConfig configuration, Environment environment)
StreamTaskStateInitializer
createStreamTaskStateInitializer(SubTaskInitializationMetricsBuilder initializationMetrics)
protected void
declineCheckpoint(long checkpointId)
void
dispatchOperatorEvent(OperatorID operator, SerializedValue<OperatorEvent> event)
protected void
endData(StopMode mode)
protected void
finalize()
The finalize method shuts down the timer.protected long
getAsyncCheckpointStartDelayNanos()
ExecutorService
getAsyncOperationsThreadPool()
CloseableRegistry
getCancelables()
StreamTask.CanEmitBatchOfRecordsChecker
getCanEmitBatchOfRecords()
protected Optional<CheckpointBarrierHandler>
getCheckpointBarrierHandler()
Acquires the optionalCheckpointBarrierHandler
associated with this stream task.CheckpointStorageWorkerView
getCheckpointStorage()
protected CompletableFuture<Void>
getCompletionFuture()
StreamConfig
getConfiguration()
Environment
getEnvironment()
MailboxExecutorFactory
getMailboxExecutorFactory()
String
getName()
Gets the name of the task, in the form "taskname (2/5)".ProcessingTimeServiceFactory
getProcessingTimeServiceFactory()
void
handleAsyncException(String message, Throwable exception)
Handles an exception thrown by another thread (e.g. a TriggerTask), other than the one executing the main task by failing the task entirely.boolean
hasMail()
protected abstract void
init()
void
invoke()
Starts the execution.boolean
isCanceled()
boolean
isFailing()
boolean
isMailboxLoopRunning()
boolean
isRunning()
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.protected void
notifyEndOfData()
protected void
processInput(MailboxDefaultAction.Controller controller)
This method implements the default action of the task (e.g. processing one event from the input).void
restore()
This method can be called beforeTaskInvokable.invoke()
to restore an invokable object for the last valid state, if it has it.void
runMailboxLoop()
boolean
runMailboxStep()
boolean
runSingleMailboxLoop()
protected void
setSynchronousSavepoint(long checkpointId)
protected Counter
setupNumRecordsInCounter(StreamOperator streamOperator)
String
toString()
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.-
Methods inherited from class java.lang.Object
clone, equals, getClass, hashCode, notify, notifyAll, wait, wait, wait
-
Methods inherited from interface org.apache.flink.streaming.runtime.tasks.ContainingTaskDetails
getExecutionConfig, getIndexInSubtaskGroup, getJobConfiguration, getUserCodeClassLoader
-
-
-
-
Field Detail
-
TRIGGER_THREAD_GROUP
public static final ThreadGroup TRIGGER_THREAD_GROUP
The thread group that holds all trigger timer threads.
-
LOG
protected static final org.slf4j.Logger LOG
The logger used by the StreamTask and its subclasses.
-
inputProcessor
@Nullable protected StreamInputProcessor inputProcessor
The input processor. Initialized ininit()
method.
-
mainOperator
protected OP extends StreamOperator<OUT> mainOperator
the main operator that consumes the input streams of this task.
-
operatorChain
protected OperatorChain<OUT,OP extends StreamOperator<OUT>> operatorChain
The chain of operators executed by this task.
-
configuration
protected final StreamConfig configuration
The configuration of this streaming task.
-
stateBackend
protected final StateBackend stateBackend
Our state backend. We use this to create a keyed state backend.
-
checkpointStorage
protected final CheckpointStorage checkpointStorage
Our checkpoint storage. We use this to create checkpoint streams.
-
timerService
protected final TimerService timerService
The internalTimerService
used to define the current processing time (default =System.currentTimeMillis()
) and register timers for tasks to be executed in the future.
-
systemTimerService
protected final TimerService systemTimerService
In contrast totimerService
we should not register any user timers here. It should be used only for system level timers.
-
recordWriter
protected final RecordWriterDelegate<SerializationDelegate<StreamRecord<OUT>>> recordWriter
-
mailboxProcessor
protected final MailboxProcessor mailboxProcessor
-
-
Constructor Detail
-
StreamTask
protected StreamTask(Environment env) throws Exception
Constructor for initialization, possibly with initial state (recovery / savepoint / etc).- Parameters:
env
- The task environment for this task.- Throws:
Exception
-
StreamTask
protected StreamTask(Environment env, @Nullable TimerService timerService) throws Exception
Constructor for initialization, possibly with initial state (recovery / savepoint / etc).- Parameters:
env
- The task environment for this task.timerService
- Optionally, a specific timer service to use.- Throws:
Exception
-
StreamTask
protected StreamTask(Environment environment, @Nullable TimerService timerService, Thread.UncaughtExceptionHandler uncaughtExceptionHandler) throws Exception
- Throws:
Exception
-
StreamTask
protected StreamTask(Environment environment, @Nullable TimerService timerService, Thread.UncaughtExceptionHandler uncaughtExceptionHandler, StreamTaskActionExecutor actionExecutor) throws Exception
Constructor for initialization, possibly with initial state (recovery / savepoint / etc).This constructor accepts a special
TimerService
. By default (and if null is passes for the timer service) aDefaultTimerService
will be used.- Parameters:
environment
- The task environment for this task.timerService
- Optionally, a specific timer service to use.uncaughtExceptionHandler
- to handle uncaught exceptions in the async operations thread poolactionExecutor
- a mean to wrap all actions performed by this task thread. Currently, only SynchronizedActionExecutor can be used to preserve locking semantics.- Throws:
Exception
-
StreamTask
protected StreamTask(Environment environment, @Nullable TimerService timerService, Thread.UncaughtExceptionHandler uncaughtExceptionHandler, StreamTaskActionExecutor actionExecutor, TaskMailbox mailbox) throws Exception
- Throws:
Exception
-
-
Method Detail
-
processInput
protected void processInput(MailboxDefaultAction.Controller controller) throws Exception
This method implements the default action of the task (e.g. processing one event from the input). Implementations should (in general) be non-blocking.- Parameters:
controller
- controller object for collaborative interaction between the action and the stream task.- Throws:
Exception
- on any problems in the action.
-
notifyEndOfData
protected void notifyEndOfData()
-
setSynchronousSavepoint
protected void setSynchronousSavepoint(long checkpointId)
-
advanceToEndOfEventTime
protected void advanceToEndOfEventTime() throws Exception
Emits theMAX_WATERMARK
so that all registered timers are fired.This is used by the source task when the job is
TERMINATED
. In the case, we want all the timers registered throughout the pipeline to fire and the related state (e.g. windows) to be flushed.For tasks other than the source task, this method does nothing.
- Throws:
Exception
-
createStreamTaskStateInitializer
public StreamTaskStateInitializer createStreamTaskStateInitializer(SubTaskInitializationMetricsBuilder initializationMetrics)
-
setupNumRecordsInCounter
protected Counter setupNumRecordsInCounter(StreamOperator streamOperator)
-
restore
public final 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
-
invoke
public final 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
-
runSingleMailboxLoop
@VisibleForTesting public boolean runSingleMailboxLoop() throws Exception
- Throws:
Exception
-
runMailboxStep
@VisibleForTesting public boolean runMailboxStep() throws Exception
- Throws:
Exception
-
isMailboxLoopRunning
@VisibleForTesting public boolean isMailboxLoopRunning()
-
cleanUp
public final void cleanUp(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
-
getCompletionFuture
protected CompletableFuture<Void> getCompletionFuture()
-
cancel
public final 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
-
getMailboxExecutorFactory
public MailboxExecutorFactory getMailboxExecutorFactory()
-
hasMail
public boolean hasMail()
-
getCanEmitBatchOfRecords
public StreamTask.CanEmitBatchOfRecordsChecker getCanEmitBatchOfRecords()
-
isRunning
public final boolean isRunning()
-
isCanceled
public final boolean isCanceled()
-
isFailing
public final boolean isFailing()
-
finalize
protected void finalize() throws Throwable
The finalize method shuts down the timer. This is a fail-safe shutdown, in case the original shutdown method was never called.This should not be relied upon! It will cause shutdown to happen much later than if manual shutdown is attempted, and cause threads to linger for longer than needed.
-
getName
public final String getName()
Gets the name of the task, in the form "taskname (2/5)".- Returns:
- The name of the task.
-
getCheckpointStorage
public CheckpointStorageWorkerView getCheckpointStorage()
-
getConfiguration
public StreamConfig getConfiguration()
-
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
-
getCheckpointBarrierHandler
protected Optional<CheckpointBarrierHandler> getCheckpointBarrierHandler()
Acquires the optionalCheckpointBarrierHandler
associated with this stream task. TheCheckpointBarrierHandler
should exist if the task has data inputs and requires to align the barriers.
-
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
-
declineCheckpoint
protected void declineCheckpoint(long checkpointId)
-
getAsyncOperationsThreadPool
public final ExecutorService getAsyncOperationsThreadPool()
-
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
-
getProcessingTimeServiceFactory
public ProcessingTimeServiceFactory getProcessingTimeServiceFactory()
-
handleAsyncException
public void handleAsyncException(String message, Throwable exception)
Handles an exception thrown by another thread (e.g. a TriggerTask), other than the one executing the main task by failing the task entirely.In more detail, it marks task execution failed for an external reason (a reason other than the task code itself throwing an exception). If the task is already in a terminal state (such as FINISHED, CANCELED, FAILED), or if the task is already canceling this does nothing. Otherwise it sets the state to FAILED, and, if the invokable code is running, starts an asynchronous thread that aborts that code.
This method never blocks.
- Specified by:
handleAsyncException
in interfaceAsyncExceptionHandler
-
getCancelables
public final CloseableRegistry getCancelables()
-
createRecordWriterDelegate
@VisibleForTesting public static <OUT> RecordWriterDelegate<SerializationDelegate<StreamRecord<OUT>>> createRecordWriterDelegate(StreamConfig configuration, Environment environment)
-
getAsyncCheckpointStartDelayNanos
protected long getAsyncCheckpointStartDelayNanos()
-
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.
-
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()
-
-