Interface MailboxExecutor
-
@PublicEvolving public interface MailboxExecutorExecutorlike interface for a build around a mailbox-based execution model.MailboxExecutorcan 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 } } }
-
-
Field Summary
Fields Modifier and Type Field Description static Object[]EMPTY_ARGSA constant for empty args to save on object allocation.
-
Method Summary
All Methods Instance Methods Abstract Methods Default Methods Modifier and Type Method Description default voidexecute(ThrowingRunnable<? extends Exception> command, String description)Executes the given command at some time in the future in the mailbox thread.voidexecute(ThrowingRunnable<? extends Exception> command, String descriptionFormat, Object... descriptionArgs)Executes the given command at some time in the future in the mailbox thread.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.booleantryYield()This method attempts to run the command at the head of the mailbox.voidyield()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.
-
-
-
Field Detail
-
EMPTY_ARGS
static final Object[] EMPTY_ARGS
A constant for empty args to save on object allocation.
-
-
Method Detail
-
execute
default void execute(ThrowingRunnable<? extends Exception> command, String description)
Executes the given command at some time in the future in the mailbox thread.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
Formattersyntax. The actual description is only formatted on demand.- Parameters:
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.- Throws:
RejectedExecutionException- if this task cannot be accepted for execution, e.g. because the mailbox is quiesced or closed.
-
execute
void execute(ThrowingRunnable<? extends Exception> command, String descriptionFormat, Object... descriptionArgs)
Executes the given command at some time in the future in the mailbox thread.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
Formattersyntax. The actual description is only formatted on demand.- Parameters:
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.- Throws:
RejectedExecutionException- if this task cannot be accepted for execution, e.g. because the mailbox is quiesced or closed.
-
submit
@Nonnull default Future<Void> submit(@Nonnull 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. The Future'sgetmethod will returnnullupon 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
Formattersyntax. The actual description is only formatted on demand.- Parameters:
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.- Returns:
- a Future representing pending completion of the task
- Throws:
RejectedExecutionException- if this task cannot be accepted for execution, e.g. because the mailbox is quiesced or closed.
-
submit
@Nonnull default Future<Void> submit(@Nonnull RunnableWithException command, String description)
Submits the given command for execution in the future in the mailbox thread and returns a Future representing that command. The Future'sgetmethod will returnnullupon 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
Formattersyntax. The actual description is only formatted on demand.- Parameters:
command- the command to submitdescription- the optional description for the command that is used for debugging and error-reporting.- Returns:
- a Future representing pending completion of the task
- Throws:
RejectedExecutionException- if this task cannot be accepted for execution, e.g. because the mailbox is quiesced or closed.
-
submit
@Nonnull default <T> Future<T> submit(@Nonnull 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. The Future'sgetmethod will returnnullupon 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
Formattersyntax. The actual description is only formatted on demand.- Parameters:
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.- Returns:
- a Future representing pending completion of the task
- Throws:
RejectedExecutionException- if this task cannot be accepted for execution, e.g. because the mailbox is quiesced or closed.
-
submit
@Nonnull default <T> Future<T> submit(@Nonnull 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. The Future'sgetmethod will returnnullupon 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
Formattersyntax. The actual description is only formatted on demand.- Parameters:
command- the command to submitdescription- the optional description for the command that is used for debugging and error-reporting.- Returns:
- a Future representing pending completion of the task
- Throws:
RejectedExecutionException- if this task cannot be accepted for execution, e.g. because the mailbox is quiesced or closed.
-
yield
void yield() throws InterruptedException, FlinkRuntimeExceptionThis 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. The method blocks until another command to run is available in the mailbox and must only be called from the mailbox thread. Must only be called from the mailbox thread to not violate the single-threaded execution model.- Throws:
InterruptedException- on interruption.IllegalStateException- if the mailbox is closed and can no longer supply runnables for yielding.FlinkRuntimeException- if executedRunnableWithExceptionthrown an exception.
-
tryYield
boolean tryYield() throws FlinkRuntimeExceptionThis method attempts to run the command at the head of the mailbox. This is intended to be used by the mailbox thread to yield from a currently ongoing action to another command. The method returns true if a command was found and executed or false if the mailbox was empty. Must only be called from the mailbox thread to not violate the single-threaded execution model.- Returns:
- true on successful yielding to another command, false if there was no command to yield to.
- Throws:
IllegalStateException- if the mailbox is closed and can no longer supply runnables for yielding.RuntimeException- if executedRunnableWithExceptionthrown an exception.FlinkRuntimeException
-
-