public class FutureUtils extends Object
CompletableFuture
.Modifier and Type | Class and Description |
---|---|
static class |
FutureUtils.ConjunctFuture<T>
A future that is complete once multiple other futures completed.
|
static class |
FutureUtils.RetryException
Exception with which the returned future is completed if the
retry(Supplier, int,
Executor) operation fails. |
Constructor and Description |
---|
FutureUtils() |
Modifier and Type | Method and Description |
---|---|
static void |
assertNoException(CompletableFuture<?> completableFuture)
Asserts that the given
CompletableFuture is not completed exceptionally. |
static <T> FutureUtils.ConjunctFuture<Collection<T>> |
combineAll(Collection<? extends CompletableFuture<? extends T>> futures)
Creates a future that is complete once multiple other futures completed.
|
static FutureUtils.ConjunctFuture<Void> |
completeAll(Collection<? extends CompletableFuture<?>> futuresToComplete)
Creates a
FutureUtils.ConjunctFuture which is only completed after all given futures have
completed. |
static <T> CompletableFuture<T> |
completedExceptionally(Throwable cause)
Returns an exceptionally completed
CompletableFuture . |
static CompletableFuture<Void> |
completedVoidFuture()
Returns a completed future of type
Void . |
static CompletableFuture<Void> |
composeAfterwards(CompletableFuture<?> future,
java.util.function.Supplier<CompletableFuture<?>> composedAction)
Run the given asynchronous action after the completion of the given future.
|
static <T> void |
forward(CompletableFuture<T> source,
CompletableFuture<T> target)
Forwards the value from the source future to the target future.
|
static <T> void |
forwardAsync(CompletableFuture<T> source,
CompletableFuture<T> target,
Executor executor)
Forwards the value from the source future to the target future using the provided executor.
|
static <T> T |
getWithoutException(CompletableFuture<T> future)
Gets the result of a completable future without any exception thrown.
|
static <IN,OUT> CompletableFuture<OUT> |
handleAsyncIfNotDone(CompletableFuture<IN> completableFuture,
Executor executor,
java.util.function.BiFunction<? super IN,Throwable,? extends OUT> handler)
This function takes a
CompletableFuture and a handler function for the result of this
future. |
static void |
handleUncaughtException(CompletableFuture<?> completableFuture,
Thread.UncaughtExceptionHandler uncaughtExceptionHandler)
Checks that the given
CompletableFuture is not completed exceptionally. |
static <T> CompletableFuture<T> |
orTimeout(CompletableFuture<T> future,
long timeout,
TimeUnit timeUnit)
Times the given future out after the timeout.
|
static <T> CompletableFuture<T> |
orTimeout(CompletableFuture<T> future,
long timeout,
TimeUnit timeUnit,
Executor timeoutFailExecutor)
Times the given future out after the timeout.
|
static <T> CompletableFuture<T> |
retry(java.util.function.Supplier<CompletableFuture<T>> operation,
int retries,
Executor executor)
Retry the given operation the given number of times in case of a failure.
|
static <T> CompletableFuture<T> |
retrySuccessfulWithDelay(java.util.function.Supplier<CompletableFuture<T>> operation,
Time retryDelay,
Deadline deadline,
java.util.function.Predicate<T> acceptancePredicate,
ScheduledExecutor scheduledExecutor)
Retry the given operation with the given delay in between successful completions where the
result does not match a given predicate.
|
static <T> CompletableFuture<T> |
retryWithDelay(java.util.function.Supplier<CompletableFuture<T>> operation,
int retries,
Time retryDelay,
java.util.function.Predicate<Throwable> retryPredicate,
ScheduledExecutor scheduledExecutor)
Retry the given operation with the given delay in between failures.
|
static <T> CompletableFuture<T> |
retryWithDelay(java.util.function.Supplier<CompletableFuture<T>> operation,
int retries,
Time retryDelay,
ScheduledExecutor scheduledExecutor)
Retry the given operation with the given delay in between failures.
|
static CompletableFuture<Void> |
runAfterwards(CompletableFuture<?> future,
RunnableWithException runnable)
Run the given action after the completion of the given future.
|
static CompletableFuture<Void> |
runAfterwardsAsync(CompletableFuture<?> future,
RunnableWithException runnable)
Run the given action after the completion of the given future.
|
static CompletableFuture<Void> |
runAfterwardsAsync(CompletableFuture<?> future,
RunnableWithException runnable,
Executor executor)
Run the given action after the completion of the given future.
|
static <T> T |
runIfNotDoneAndGet(RunnableFuture<T> future)
Run the given
RunnableFuture if it is not done, and then retrieves its result. |
static <T> CompletableFuture<T> |
runSync(Callable<T> operation)
Fakes asynchronous execution by immediately executing the operation and returns a
(exceptionally) completed future.
|
static CompletableFuture<Void> |
scheduleWithDelay(Runnable operation,
Time delay,
ScheduledExecutor scheduledExecutor)
Schedule the operation with the given delay.
|
static <T> CompletableFuture<T> |
scheduleWithDelay(java.util.function.Supplier<T> operation,
Time delay,
ScheduledExecutor scheduledExecutor)
Schedule the operation with the given delay.
|
static <T> CompletableFuture<T> |
supplyAsync(SupplierWithException<T,?> supplier,
Executor executor)
Returns a future which is completed with the result of the
SupplierWithException . |
static <IN> CompletableFuture<Void> |
thenAcceptAsyncIfNotDone(CompletableFuture<IN> completableFuture,
Executor executor,
java.util.function.Consumer<? super IN> consumer)
This function takes a
CompletableFuture and a consumer to accept the result of this
future. |
static <IN,OUT> CompletableFuture<OUT> |
thenApplyAsyncIfNotDone(CompletableFuture<IN> completableFuture,
Executor executor,
java.util.function.Function<? super IN,? extends OUT> applyFun)
This function takes a
CompletableFuture and a function to apply to this future. |
static <IN,OUT> CompletableFuture<OUT> |
thenComposeAsyncIfNotDone(CompletableFuture<IN> completableFuture,
Executor executor,
java.util.function.Function<? super IN,? extends CompletionStage<OUT>> composeFun)
This function takes a
CompletableFuture and a function to compose with this future. |
static void |
throwIfCompletedExceptionally(CompletableFuture<?> future)
Throws the causing exception if the given future is completed exceptionally, otherwise do
nothing.
|
static java.time.Duration |
toDuration(Time time)
Converts Flink time into a
Duration . |
static <T,U extends T> |
toJava(scala.concurrent.Future<U> scalaFuture)
Converts a Scala
Future to a CompletableFuture . |
static FutureUtils.ConjunctFuture<Void> |
waitForAll(Collection<? extends CompletableFuture<?>> futures)
Creates a future that is complete once all of the given futures have completed.
|
static <IN> CompletableFuture<IN> |
whenCompleteAsyncIfNotDone(CompletableFuture<IN> completableFuture,
Executor executor,
java.util.function.BiConsumer<? super IN,? super Throwable> whenCompleteFun)
This function takes a
CompletableFuture and a bi-consumer to call on completion of
this future. |
public static CompletableFuture<Void> completedVoidFuture()
Void
.Void
public static <T> CompletableFuture<T> runSync(Callable<T> operation)
T
- type of the resultoperation
- to executedpublic static <T> CompletableFuture<T> retry(java.util.function.Supplier<CompletableFuture<T>> operation, int retries, Executor executor)
T
- type of the resultoperation
- to executedretries
- if the operation failedexecutor
- to use to run the futuresFutureUtils.RetryException
public static <T> CompletableFuture<T> retryWithDelay(java.util.function.Supplier<CompletableFuture<T>> operation, int retries, Time retryDelay, java.util.function.Predicate<Throwable> retryPredicate, ScheduledExecutor scheduledExecutor)
T
- type of the resultoperation
- to retryretries
- number of retriesretryDelay
- delay between retriesretryPredicate
- Predicate to test whether an exception is retryablescheduledExecutor
- executor to be used for the retry operationpublic static <T> CompletableFuture<T> retryWithDelay(java.util.function.Supplier<CompletableFuture<T>> operation, int retries, Time retryDelay, ScheduledExecutor scheduledExecutor)
T
- type of the resultoperation
- to retryretries
- number of retriesretryDelay
- delay between retriesscheduledExecutor
- executor to be used for the retry operationpublic static CompletableFuture<Void> scheduleWithDelay(Runnable operation, Time delay, ScheduledExecutor scheduledExecutor)
operation
- to scheduledelay
- delay to schedulescheduledExecutor
- executor to be used for the operationpublic static <T> CompletableFuture<T> scheduleWithDelay(java.util.function.Supplier<T> operation, Time delay, ScheduledExecutor scheduledExecutor)
T
- type of the resultoperation
- to scheduledelay
- delay to schedulescheduledExecutor
- executor to be used for the operationpublic static <T> CompletableFuture<T> retrySuccessfulWithDelay(java.util.function.Supplier<CompletableFuture<T>> operation, Time retryDelay, Deadline deadline, java.util.function.Predicate<T> acceptancePredicate, ScheduledExecutor scheduledExecutor)
T
- type of the resultoperation
- to retryretryDelay
- delay between retriesdeadline
- A deadline that specifies at what point we should stop retryingacceptancePredicate
- Predicate to test whether the result is acceptablescheduledExecutor
- executor to be used for the retry operationpublic static <T> CompletableFuture<T> orTimeout(CompletableFuture<T> future, long timeout, TimeUnit timeUnit)
T
- type of the given futurefuture
- to time outtimeout
- after which the given future is timed outtimeUnit
- time unit of the timeoutpublic static <T> CompletableFuture<T> orTimeout(CompletableFuture<T> future, long timeout, TimeUnit timeUnit, Executor timeoutFailExecutor)
T
- type of the given futurefuture
- to time outtimeout
- after which the given future is timed outtimeUnit
- time unit of the timeouttimeoutFailExecutor
- executor that will complete the future exceptionally after the
timeout is reachedpublic static <T> T runIfNotDoneAndGet(RunnableFuture<T> future) throws ExecutionException, InterruptedException
RunnableFuture
if it is not done, and then retrieves its result.T
- type of the resultfuture
- to run if not done and getExecutionException
- if a problem occurredInterruptedException
- if the current thread has been interruptedpublic static CompletableFuture<Void> runAfterwards(CompletableFuture<?> future, RunnableWithException runnable)
future
- to wait for its completionrunnable
- action which is triggered after the future's completionpublic static CompletableFuture<Void> runAfterwardsAsync(CompletableFuture<?> future, RunnableWithException runnable)
future
- to wait for its completionrunnable
- action which is triggered after the future's completionpublic static CompletableFuture<Void> runAfterwardsAsync(CompletableFuture<?> future, RunnableWithException runnable, Executor executor)
future
- to wait for its completionrunnable
- action which is triggered after the future's completionexecutor
- to run the given actionpublic static CompletableFuture<Void> composeAfterwards(CompletableFuture<?> future, java.util.function.Supplier<CompletableFuture<?>> composedAction)
future
- to wait for its completioncomposedAction
- asynchronous action which is triggered after the future's completionpublic static <T> FutureUtils.ConjunctFuture<Collection<T>> combineAll(Collection<? extends CompletableFuture<? extends T>> futures)
The ConjunctFuture gives access to how many Futures in the conjunction have already
completed successfully, via FutureUtils.ConjunctFuture.getNumFuturesCompleted()
.
futures
- The futures that make up the conjunction. No null entries are allowed.public static FutureUtils.ConjunctFuture<Void> waitForAll(Collection<? extends CompletableFuture<?>> futures)
The ConjunctFuture gives access to how many Futures have already completed successfully,
via FutureUtils.ConjunctFuture.getNumFuturesCompleted()
.
futures
- The futures to wait on. No null entries are allowed.public static FutureUtils.ConjunctFuture<Void> completeAll(Collection<? extends CompletableFuture<?>> futuresToComplete)
FutureUtils.ConjunctFuture
which is only completed after all given futures have
completed. Unlike waitForAll(Collection)
, the resulting future won't be
completed directly if one of the given futures is completed exceptionally. Instead, all
occurring exception will be collected and combined to a single exception. If at least on
exception occurs, then the resulting future will be completed exceptionally.futuresToComplete
- futures to completepublic static <T> CompletableFuture<T> completedExceptionally(Throwable cause)
CompletableFuture
.T
- type of the futurecause
- to complete the future withpublic static <T> CompletableFuture<T> supplyAsync(SupplierWithException<T,?> supplier, Executor executor)
SupplierWithException
.T
- type of the resultsupplier
- to provide the future's valueexecutor
- to execute the supplierpublic static java.time.Duration toDuration(Time time)
Duration
.time
- to convert into a Durationpublic static <T,U extends T> CompletableFuture<T> toJava(scala.concurrent.Future<U> scalaFuture)
Future
to a CompletableFuture
.T
- type of the future valueU
- type of the original futurescalaFuture
- to convert to a Java 8 CompletableFuturepublic static <IN,OUT> CompletableFuture<OUT> thenApplyAsyncIfNotDone(CompletableFuture<IN> completableFuture, Executor executor, java.util.function.Function<? super IN,? extends OUT> applyFun)
CompletableFuture
and a function to apply to this future. If
the input future is already done, this function returns CompletableFuture.thenApply(Function)
. Otherwise, the return value is CompletableFuture.thenApplyAsync(Function, Executor)
with the given executor.IN
- type of the input future.OUT
- type of the output future.completableFuture
- the completable future for which we want to apply.executor
- the executor to run the apply function if the future is not yet done.applyFun
- the function to apply.public static <IN,OUT> CompletableFuture<OUT> thenComposeAsyncIfNotDone(CompletableFuture<IN> completableFuture, Executor executor, java.util.function.Function<? super IN,? extends CompletionStage<OUT>> composeFun)
CompletableFuture
and a function to compose with this future.
If the input future is already done, this function returns CompletableFuture.thenCompose(Function)
. Otherwise, the return value is CompletableFuture.thenComposeAsync(Function, Executor)
with the given executor.IN
- type of the input future.OUT
- type of the output future.completableFuture
- the completable future for which we want to compose.executor
- the executor to run the compose function if the future is not yet done.composeFun
- the function to compose.public static <IN> CompletableFuture<IN> whenCompleteAsyncIfNotDone(CompletableFuture<IN> completableFuture, Executor executor, java.util.function.BiConsumer<? super IN,? super Throwable> whenCompleteFun)
CompletableFuture
and a bi-consumer to call on completion of
this future. If the input future is already done, this function returns CompletableFuture.whenComplete(BiConsumer)
. Otherwise, the return value is CompletableFuture.whenCompleteAsync(BiConsumer, Executor)
with the given executor.IN
- type of the input future.completableFuture
- the completable future for which we want to call #whenComplete.executor
- the executor to run the whenComplete function if the future is not yet done.whenCompleteFun
- the bi-consumer function to call when the future is completed.public static <IN> CompletableFuture<Void> thenAcceptAsyncIfNotDone(CompletableFuture<IN> completableFuture, Executor executor, java.util.function.Consumer<? super IN> consumer)
CompletableFuture
and a consumer to accept the result of this
future. If the input future is already done, this function returns CompletableFuture.thenAccept(Consumer)
. Otherwise, the return value is CompletableFuture.thenAcceptAsync(Consumer, Executor)
with the given executor.IN
- type of the input future.completableFuture
- the completable future for which we want to call #thenAccept.executor
- the executor to run the thenAccept function if the future is not yet done.consumer
- the consumer function to call when the future is completed.public static <IN,OUT> CompletableFuture<OUT> handleAsyncIfNotDone(CompletableFuture<IN> completableFuture, Executor executor, java.util.function.BiFunction<? super IN,Throwable,? extends OUT> handler)
CompletableFuture
and a handler function for the result of this
future. If the input future is already done, this function returns CompletableFuture.handle(BiFunction)
. Otherwise, the return value is CompletableFuture.handleAsync(BiFunction, Executor)
with the given executor.IN
- type of the handler input argument.OUT
- type of the handler return value.completableFuture
- the completable future for which we want to call #handle.executor
- the executor to run the handle function if the future is not yet done.handler
- the handler function to call when the future is completed.@Nullable public static <T> T getWithoutException(CompletableFuture<T> future)
T
- the type of resultfuture
- the completable future specified.public static void assertNoException(CompletableFuture<?> completableFuture)
CompletableFuture
is not completed exceptionally. If the
future is completed exceptionally, then it will call the FatalExitExceptionHandler
.completableFuture
- to assert for no exceptionspublic static void handleUncaughtException(CompletableFuture<?> completableFuture, Thread.UncaughtExceptionHandler uncaughtExceptionHandler)
CompletableFuture
is not completed exceptionally. If the future
is completed exceptionally, then it will call the given uncaught exception handler.completableFuture
- to assert for no exceptionsuncaughtExceptionHandler
- to call if the future is completed exceptionallypublic static <T> void forward(CompletableFuture<T> source, CompletableFuture<T> target)
T
- type of the valuesource
- future to forward the value fromtarget
- future to forward the value topublic static <T> void forwardAsync(CompletableFuture<T> source, CompletableFuture<T> target, Executor executor)
T
- type of the valuesource
- future to forward the value fromtarget
- future to forward the value toexecutor
- executor to forward the source value to the target futurepublic static void throwIfCompletedExceptionally(CompletableFuture<?> future) throws Exception
future
- the future to check.Exception
- when the future is completed exceptionally.Copyright © 2014–2021 The Apache Software Foundation. All rights reserved.