@PublicEvolving public interface MailboxExecutor
Executor
like interface for a build around a mailbox-based execution
model. MailboxExecutor
can also execute downstream messages of a mailbox by yielding
control from the task thread.
All submission functions can be called from any thread and will enqueue the action for further processing in a FIFO fashion.
The yielding functions avoid the following situation: One operator cannot fully process an input record and blocks the task thread until some resources are available. However, since the introduction of the mailbox model blocking the task thread will not only block new inputs but also all events from being processed. If the resources depend on downstream operators being able to process such events (e.g., timers), then we may easily arrive at some livelocks.
The yielding functions will only process events from the operator itself and any downstream operator. Events of upstream operators are only processed when the input has been fully processed or if they yield themselves. This method avoid congestion and potential deadlocks, but will process mails slightly out-of-order, effectively creating a view on the mailbox that contains no message from upstream operators.
All yielding functions must be called in the mailbox thread to not violate the single-threaded execution model. There are two typical cases, both waiting until the resource is available. The main difference is if the resource becomes available through a mailbox message itself or not.
If the resource becomes available through a mailbox mail, we can effectively block the task thread. Implicitly, this requires the mail to be enqueued by a different thread.
while (resource not available) {
mailboxExecutor.yield();
}
in some other thread
mailboxExecutor.execute(() -> free resource, "freeing resource");
If the resource becomes available through an external mechanism or the corresponding mail needs to be enqueued in the task thread, we cannot block.
while (resource not available) {
if (!mailboxExecutor.tryYield()) {
// do stuff or sleep for a small amount of time
if (special condition) {
free resource
}
}
}
Modifier and Type | Interface and Description |
---|---|
static interface |
MailboxExecutor.MailOptions
Extra options to configure enqueued mails.
|
Modifier and Type | Field and Description |
---|---|
static Object[] |
EMPTY_ARGS
A constant for empty args to save on object allocation.
|
Modifier and Type | Method and Description |
---|---|
default void |
execute(MailboxExecutor.MailOptions mailOptions,
ThrowingRunnable<? extends Exception> command,
String description)
Executes the given command at some time in the future in the mailbox thread.
|
void |
execute(MailboxExecutor.MailOptions mailOptions,
ThrowingRunnable<? extends Exception> command,
String descriptionFormat,
Object... descriptionArgs)
Executes the given command at some time in the future in the mailbox thread.
|
default void |
execute(ThrowingRunnable<? extends Exception> command,
String description)
Executes the given command at some time in the future in the mailbox thread.
|
default void |
execute(ThrowingRunnable<? extends Exception> command,
String descriptionFormat,
Object... descriptionArgs)
Executes the given command at some time in the future in the mailbox thread.
|
boolean |
shouldInterrupt()
Return if operator/function should interrupt a longer computation and return from the
currently processed elemenent/watermark, for example in order to let Flink perform a
checkpoint.
|
default <T> Future<T> |
submit(Callable<T> command,
String description)
Submits the given command for execution in the future in the mailbox thread and returns a
Future representing that command.
|
default <T> Future<T> |
submit(Callable<T> command,
String descriptionFormat,
Object... descriptionArgs)
Submits the given command for execution in the future in the mailbox thread and returns a
Future representing that command.
|
default Future<Void> |
submit(RunnableWithException command,
String description)
Submits the given command for execution in the future in the mailbox thread and returns a
Future representing that command.
|
default Future<Void> |
submit(RunnableWithException command,
String descriptionFormat,
Object... descriptionArgs)
Submits the given command for execution in the future in the mailbox thread and returns a
Future representing that command.
|
boolean |
tryYield()
This method attempts to run the command at the head of the mailbox.
|
void |
yield()
This method starts running the command at the head of the mailbox and is intended to be used
by the mailbox thread to yield from a currently ongoing action to another command.
|
static final Object[] EMPTY_ARGS
default void execute(ThrowingRunnable<? extends Exception> command, String description)
An optional description can (and should) be added to ease debugging and error-reporting.
The description may contain placeholder that refer to the provided description arguments
using Formatter
syntax. The actual description is only formatted on demand.
command
- the runnable task to add to the mailbox for execution.description
- the optional description for the command that is used for debugging and
error-reporting.RejectedExecutionException
- if this task cannot be accepted for execution, e.g.
because the mailbox is quiesced or closed.default void execute(MailboxExecutor.MailOptions mailOptions, ThrowingRunnable<? extends Exception> command, String description)
An optional description can (and should) be added to ease debugging and error-reporting.
The description may contain placeholder that refer to the provided description arguments
using Formatter
syntax. The actual description is only formatted on demand.
mailOptions
- additional options to configure behaviour of the command
command
- the runnable task to add to the mailbox for execution.description
- the optional description for the command that is used for debugging and
error-reporting.RejectedExecutionException
- if this task cannot be accepted for execution, e.g.
because the mailbox is quiesced or closed.default void execute(ThrowingRunnable<? extends Exception> command, String descriptionFormat, Object... descriptionArgs)
An optional description can (and should) be added to ease debugging and error-reporting.
The description may contain placeholder that refer to the provided description arguments
using Formatter
syntax. The actual description is only formatted on demand.
command
- the runnable task to add to the mailbox for execution.descriptionFormat
- the optional description for the command that is used for debugging
and error-reporting.descriptionArgs
- the parameters used to format the final description string.RejectedExecutionException
- if this task cannot be accepted for execution, e.g.
because the mailbox is quiesced or closed.void execute(MailboxExecutor.MailOptions mailOptions, ThrowingRunnable<? extends Exception> command, String descriptionFormat, Object... descriptionArgs)
An optional description can (and should) be added to ease debugging and error-reporting.
The description may contain placeholder that refer to the provided description arguments
using Formatter
syntax. The actual description is only formatted on demand.
mailOptions
- additional options to configure behaviour of the command
command
- the runnable task to add to the mailbox for execution.descriptionFormat
- the optional description for the command that is used for debugging
and error-reporting.descriptionArgs
- the parameters used to format the final description string.RejectedExecutionException
- if this task cannot be accepted for execution, e.g.
because the mailbox is quiesced or closed.@Nonnull default Future<Void> submit(@Nonnull RunnableWithException command, String descriptionFormat, Object... descriptionArgs)
get
method will return null
upon successful completion.
An optional description can (and should) be added to ease debugging and error-reporting.
The description may contain placeholder that refer to the provided description arguments
using Formatter
syntax. The actual description is only formatted on demand.
command
- the command to submitdescriptionFormat
- the optional description for the command that is used for debugging
and error-reporting.descriptionArgs
- the parameters used to format the final description string.RejectedExecutionException
- if this task cannot be accepted for execution, e.g.
because the mailbox is quiesced or closed.@Nonnull default Future<Void> submit(@Nonnull RunnableWithException command, String description)
get
method will return null
upon successful completion.
An optional description can (and should) be added to ease debugging and error-reporting.
The description may contain placeholder that refer to the provided description arguments
using Formatter
syntax. The actual description is only formatted on demand.
command
- the command to submitdescription
- the optional description for the command that is used for debugging and
error-reporting.RejectedExecutionException
- if this task cannot be accepted for execution, e.g.
because the mailbox is quiesced or closed.@Nonnull default <T> Future<T> submit(@Nonnull Callable<T> command, String descriptionFormat, Object... descriptionArgs)
get
method will return null
upon successful completion.
An optional description can (and should) be added to ease debugging and error-reporting.
The description may contain placeholder that refer to the provided description arguments
using Formatter
syntax. The actual description is only formatted on demand.
command
- the command to submitdescriptionFormat
- the optional description for the command that is used for debugging
and error-reporting.descriptionArgs
- the parameters used to format the final description string.RejectedExecutionException
- if this task cannot be accepted for execution, e.g.
because the mailbox is quiesced or closed.@Nonnull default <T> Future<T> submit(@Nonnull Callable<T> command, String description)
get
method will return null
upon successful completion.
An optional description can (and should) be added to ease debugging and error-reporting.
The description may contain placeholder that refer to the provided description arguments
using Formatter
syntax. The actual description is only formatted on demand.
command
- the command to submitdescription
- the optional description for the command that is used for debugging and
error-reporting.RejectedExecutionException
- if this task cannot be accepted for execution, e.g.
because the mailbox is quiesced or closed.void yield() throws InterruptedException, FlinkRuntimeException
InterruptedException
- on interruption.IllegalStateException
- if the mailbox is closed and can no longer supply runnables for
yielding.FlinkRuntimeException
- if executed RunnableWithException
thrown an exception.boolean tryYield() throws FlinkRuntimeException
IllegalStateException
- if the mailbox is closed and can no longer supply runnables for
yielding.RuntimeException
- if executed RunnableWithException
thrown an exception.FlinkRuntimeException
boolean shouldInterrupt()
Copyright © 2014–2024 The Apache Software Foundation. All rights reserved.