2009-05-25 14:48:43 +02:00
|
|
|
/**
|
2012-01-19 18:21:06 +01:00
|
|
|
* Copyright (C) 2009-2012 Typesafe Inc. <http://www.typesafe.com>
|
2009-05-25 14:48:43 +02:00
|
|
|
*/
|
|
|
|
|
|
2010-10-26 12:49:25 +02:00
|
|
|
package akka.dispatch
|
2009-05-25 14:48:43 +02:00
|
|
|
|
2011-10-27 12:23:01 +02:00
|
|
|
import akka.event.Logging.Error
|
2011-08-26 17:25:18 +02:00
|
|
|
import scala.Option
|
2012-01-25 15:38:04 +01:00
|
|
|
import akka.japi.{ Function ⇒ JFunc, Option ⇒ JOption }
|
2011-04-21 15:12:47 -06:00
|
|
|
import scala.util.continuations._
|
2011-05-18 17:25:30 +02:00
|
|
|
import java.lang.{ Iterable ⇒ JIterable }
|
|
|
|
|
import java.util.{ LinkedList ⇒ JLinkedList }
|
2011-04-27 00:38:10 +02:00
|
|
|
import scala.annotation.tailrec
|
2011-04-28 20:53:45 -06:00
|
|
|
import scala.collection.mutable.Stack
|
2012-01-25 15:38:04 +01:00
|
|
|
import akka.util.{ Duration, BoxedType }
|
2011-12-12 22:50:08 +01:00
|
|
|
import akka.dispatch.Await.CanAwait
|
2012-02-01 17:38:12 +01:00
|
|
|
import akka.util.NonFatal
|
2012-02-03 11:15:42 +01:00
|
|
|
import akka.event.Logging.LogEventException
|
|
|
|
|
import akka.event.Logging.Debug
|
2012-02-06 12:48:03 +01:00
|
|
|
import java.util.concurrent.TimeUnit.NANOSECONDS
|
|
|
|
|
import java.util.concurrent.{ ExecutionException, Callable, TimeoutException }
|
|
|
|
|
import java.util.concurrent.atomic.{ AtomicInteger, AtomicReferenceFieldUpdater }
|
2012-02-09 22:58:28 +01:00
|
|
|
import akka.pattern.AskTimeoutException
|
2012-02-24 15:03:36 +01:00
|
|
|
import scala.util.DynamicVariable
|
|
|
|
|
import scala.runtime.BoxedUnit
|
2009-05-25 14:48:43 +02:00
|
|
|
|
2011-12-12 22:50:08 +01:00
|
|
|
object Await {
|
2012-02-03 13:59:38 +01:00
|
|
|
|
|
|
|
|
/**
|
|
|
|
|
* Internal Akka use only
|
|
|
|
|
*/
|
2011-12-12 22:50:08 +01:00
|
|
|
sealed trait CanAwait
|
2009-06-10 20:04:33 +02:00
|
|
|
|
2012-02-03 13:59:38 +01:00
|
|
|
/**
|
|
|
|
|
* Classes that implement Awaitable can be used with Await,
|
|
|
|
|
* this is used to do blocking operations (blocking in the "pause this thread" sense)
|
|
|
|
|
*/
|
2011-12-12 22:50:08 +01:00
|
|
|
trait Awaitable[+T] {
|
2011-12-11 00:40:52 +01:00
|
|
|
/**
|
2012-02-03 13:49:07 +01:00
|
|
|
* Should throw [[java.util.concurrent.TimeoutException]] if times out
|
2011-12-15 19:35:53 +01:00
|
|
|
* This method should not be called directly.
|
2011-12-11 00:40:52 +01:00
|
|
|
*/
|
2012-02-16 12:31:49 +01:00
|
|
|
@throws(classOf[TimeoutException])
|
2011-12-12 22:50:08 +01:00
|
|
|
def ready(atMost: Duration)(implicit permit: CanAwait): this.type
|
2011-12-11 01:29:46 +01:00
|
|
|
|
|
|
|
|
/**
|
|
|
|
|
* Throws exceptions if cannot produce a T within the specified time
|
2011-12-15 19:35:53 +01:00
|
|
|
* This method should not be called directly.
|
2011-12-11 01:29:46 +01:00
|
|
|
*/
|
2012-02-16 12:31:49 +01:00
|
|
|
@throws(classOf[Exception])
|
2011-12-12 22:50:08 +01:00
|
|
|
def result(atMost: Duration)(implicit permit: CanAwait): T
|
2011-12-11 00:40:52 +01:00
|
|
|
}
|
2011-02-28 16:29:03 -07:00
|
|
|
|
2011-12-19 15:05:33 +01:00
|
|
|
private[this] implicit final val permit = new CanAwait {}
|
2011-06-18 23:23:47 -06:00
|
|
|
|
2012-02-03 13:59:38 +01:00
|
|
|
/**
|
|
|
|
|
* Blocks the current Thread to wait for the given awaitable to be ready.
|
|
|
|
|
* WARNING: Blocking operation, use with caution.
|
|
|
|
|
*
|
|
|
|
|
* @throws [[java.util.concurrent.TimeoutException]] if times out
|
2012-02-06 16:59:09 +01:00
|
|
|
* @return The returned value as returned by Awaitable.ready
|
2012-02-03 13:59:38 +01:00
|
|
|
*/
|
2012-02-16 12:31:49 +01:00
|
|
|
@throws(classOf[TimeoutException])
|
2011-12-12 22:50:08 +01:00
|
|
|
def ready[T <: Awaitable[_]](awaitable: T, atMost: Duration): T = awaitable.ready(atMost)
|
2012-02-03 13:59:38 +01:00
|
|
|
|
|
|
|
|
/**
|
|
|
|
|
* Blocks the current Thread to wait for the given awaitable to have a result.
|
|
|
|
|
* WARNING: Blocking operation, use with caution.
|
|
|
|
|
*
|
|
|
|
|
* @throws [[java.util.concurrent.TimeoutException]] if times out
|
2012-02-06 16:59:09 +01:00
|
|
|
* @return The returned value as returned by Awaitable.result
|
2012-02-03 13:59:38 +01:00
|
|
|
*/
|
2012-02-16 12:31:49 +01:00
|
|
|
@throws(classOf[Exception])
|
2011-12-12 22:50:08 +01:00
|
|
|
def result[T](awaitable: Awaitable[T], atMost: Duration): T = awaitable.result(atMost)
|
2011-12-11 00:40:52 +01:00
|
|
|
}
|
|
|
|
|
|
2011-12-15 16:56:53 +01:00
|
|
|
/**
|
|
|
|
|
* Futures is the Java API for Futures and Promises
|
|
|
|
|
*/
|
2011-12-12 14:39:10 +01:00
|
|
|
object Futures {
|
2011-02-28 16:29:03 -07:00
|
|
|
|
2010-11-12 14:04:06 +01:00
|
|
|
/**
|
2011-03-22 17:20:35 +01:00
|
|
|
* Java API, equivalent to Future.apply
|
2010-11-12 14:04:06 +01:00
|
|
|
*/
|
2011-12-30 13:48:31 +01:00
|
|
|
def future[T](body: Callable[T], executor: ExecutionContext): Future[T] = Future(body.call)(executor)
|
2011-06-18 23:23:47 -06:00
|
|
|
|
2010-10-26 16:40:09 +02:00
|
|
|
/**
|
2011-12-11 00:40:52 +01:00
|
|
|
* Java API, equivalent to Promise.apply
|
2010-10-26 16:40:09 +02:00
|
|
|
*/
|
2011-12-30 13:48:31 +01:00
|
|
|
def promise[T](executor: ExecutionContext): Promise[T] = Promise[T]()(executor)
|
2010-11-12 12:54:48 +01:00
|
|
|
|
2011-12-15 16:56:53 +01:00
|
|
|
/**
|
|
|
|
|
* Java API, creates an already completed Promise with the specified exception
|
|
|
|
|
*/
|
2011-12-30 13:48:31 +01:00
|
|
|
def failed[T](exception: Throwable, executor: ExecutionContext): Promise[T] = Promise.failed(exception)(executor)
|
2011-12-15 16:56:53 +01:00
|
|
|
|
|
|
|
|
/**
|
|
|
|
|
* Java API, Creates an already completed Promise with the specified result
|
|
|
|
|
*/
|
2011-12-30 13:48:31 +01:00
|
|
|
def successful[T](result: T, executor: ExecutionContext): Promise[T] = Promise.successful(result)(executor)
|
2011-12-15 16:56:53 +01:00
|
|
|
|
2011-08-26 17:25:18 +02:00
|
|
|
/**
|
|
|
|
|
* Java API.
|
|
|
|
|
* Returns a Future that will hold the optional result of the first Future with a result that matches the predicate
|
|
|
|
|
*/
|
2011-12-30 13:48:31 +01:00
|
|
|
def find[T <: AnyRef](futures: JIterable[Future[T]], predicate: JFunc[T, java.lang.Boolean], executor: ExecutionContext): Future[JOption[T]] = {
|
|
|
|
|
Future.find[T]((scala.collection.JavaConversions.iterableAsScalaIterable(futures)))(predicate.apply(_))(executor).map(JOption.fromScalaOption(_))
|
2011-08-26 17:25:18 +02:00
|
|
|
}
|
|
|
|
|
|
2011-03-22 22:12:16 +01:00
|
|
|
/**
|
2011-03-30 21:19:26 +02:00
|
|
|
* Java API.
|
2011-03-22 22:12:16 +01:00
|
|
|
* Returns a Future to the result of the first future in the list that is completed
|
|
|
|
|
*/
|
2011-12-30 13:48:31 +01:00
|
|
|
def firstCompletedOf[T <: AnyRef](futures: JIterable[Future[T]], executor: ExecutionContext): Future[T] =
|
|
|
|
|
Future.firstCompletedOf(scala.collection.JavaConversions.iterableAsScalaIterable(futures))(executor)
|
2011-10-06 21:19:46 +02:00
|
|
|
|
2011-03-22 22:12:16 +01:00
|
|
|
/**
|
|
|
|
|
* Java API
|
2011-12-27 12:02:55 +01:00
|
|
|
* A non-blocking fold over the specified futures, with the start value of the given zero.
|
2011-03-22 22:12:16 +01:00
|
|
|
* The fold is performed on the thread where the last future is completed,
|
|
|
|
|
* the result will be the first failure of any of the futures, or any failure in the actual fold,
|
|
|
|
|
* or the result of the fold.
|
|
|
|
|
*/
|
2011-12-30 13:48:31 +01:00
|
|
|
def fold[T <: AnyRef, R <: AnyRef](zero: R, futures: JIterable[Future[T]], fun: akka.japi.Function2[R, T, R], executor: ExecutionContext): Future[R] =
|
2012-01-19 17:41:53 +01:00
|
|
|
Future.fold(scala.collection.JavaConversions.iterableAsScalaIterable(futures))(zero)(fun.apply)(executor)
|
2011-06-18 23:23:47 -06:00
|
|
|
|
2011-03-22 22:12:16 +01:00
|
|
|
/**
|
2011-03-30 21:19:26 +02:00
|
|
|
* Java API.
|
2011-03-22 22:12:16 +01:00
|
|
|
* Initiates a fold over the supplied futures where the fold-zero is the result value of the Future that's completed first
|
|
|
|
|
*/
|
2012-01-19 17:41:53 +01:00
|
|
|
def reduce[T <: AnyRef, R >: T](futures: JIterable[Future[T]], fun: akka.japi.Function2[R, T, R], executor: ExecutionContext): Future[R] =
|
|
|
|
|
Future.reduce[T, R](scala.collection.JavaConversions.iterableAsScalaIterable(futures))(fun.apply)(executor)
|
2011-10-06 21:19:46 +02:00
|
|
|
|
2011-04-15 13:09:53 -06:00
|
|
|
/**
|
|
|
|
|
* Java API.
|
2011-12-14 01:24:55 +01:00
|
|
|
* Simple version of Future.traverse. Transforms a JIterable[Future[A]] into a Future[JIterable[A]].
|
2011-04-15 13:09:53 -06:00
|
|
|
* Useful for reducing many Futures into a single Future.
|
|
|
|
|
*/
|
2011-12-30 13:48:31 +01:00
|
|
|
def sequence[A](in: JIterable[Future[A]], executor: ExecutionContext): Future[JIterable[A]] = {
|
|
|
|
|
implicit val d = executor
|
2011-05-18 17:25:30 +02:00
|
|
|
scala.collection.JavaConversions.iterableAsScalaIterable(in).foldLeft(Future(new JLinkedList[A]()))((fr, fa) ⇒
|
|
|
|
|
for (r ← fr; a ← fa) yield {
|
2011-04-15 13:09:53 -06:00
|
|
|
r add a
|
|
|
|
|
r
|
|
|
|
|
})
|
2011-10-06 21:19:46 +02:00
|
|
|
}
|
2011-01-24 16:37:08 +01:00
|
|
|
|
2011-03-30 21:19:26 +02:00
|
|
|
/**
|
2011-04-15 13:09:53 -06:00
|
|
|
* Java API.
|
2011-12-14 01:24:55 +01:00
|
|
|
* Transforms a JIterable[A] into a Future[JIterable[B]] using the provided Function A ⇒ Future[B].
|
2011-04-03 10:40:06 -06:00
|
|
|
* This is useful for performing a parallel map. For example, to apply a function to all items of a list
|
2011-04-15 13:09:53 -06:00
|
|
|
* in parallel.
|
2011-03-30 21:19:26 +02:00
|
|
|
*/
|
2011-12-30 13:48:31 +01:00
|
|
|
def traverse[A, B](in: JIterable[A], fn: JFunc[A, Future[B]], executor: ExecutionContext): Future[JIterable[B]] = {
|
|
|
|
|
implicit val d = executor
|
2011-05-18 17:25:30 +02:00
|
|
|
scala.collection.JavaConversions.iterableAsScalaIterable(in).foldLeft(Future(new JLinkedList[B]())) { (fr, a) ⇒
|
2011-04-15 13:09:53 -06:00
|
|
|
val fb = fn(a)
|
2011-12-11 00:40:52 +01:00
|
|
|
for (r ← fr; b ← fb) yield { r add b; r }
|
2011-04-15 13:09:53 -06:00
|
|
|
}
|
2011-10-06 21:19:46 +02:00
|
|
|
}
|
2012-02-10 08:20:36 +01:00
|
|
|
|
|
|
|
|
/**
|
|
|
|
|
* Signals that the current thread of execution will potentially engage
|
2012-02-20 16:37:14 +01:00
|
|
|
* an action that will take a non-trivial amount of time, perhaps by using blocking.IO or using a lot of CPU time,
|
|
|
|
|
* giving the system a chance to spawn new threads, reuse old threads or otherwise,
|
|
|
|
|
* to prevent starvation and/or unfairness.
|
2012-02-10 08:20:36 +01:00
|
|
|
*
|
|
|
|
|
* Assures that any Future tasks initiated in the current thread will be
|
|
|
|
|
* executed asynchronously, including any tasks currently queued to be
|
|
|
|
|
* executed in the current thread. This is needed if the current task may
|
|
|
|
|
* block, causing delays in executing the remaining tasks which in some
|
|
|
|
|
* cases may cause a deadlock.
|
|
|
|
|
*
|
|
|
|
|
* Usage: Call this method in a callback (map, flatMap etc also count) to a Future,
|
|
|
|
|
* if you will be doing blocking in the callback.
|
|
|
|
|
*
|
|
|
|
|
* Note: Calling 'Await.result(future)' or 'Await.ready(future)' will automatically trigger this method.
|
|
|
|
|
*
|
|
|
|
|
*/
|
|
|
|
|
def blocking(): Unit = Future.blocking()
|
2010-02-23 10:05:47 +01:00
|
|
|
}
|
|
|
|
|
|
2011-02-28 11:47:39 -07:00
|
|
|
object Future {
|
2011-04-27 01:06:08 +02:00
|
|
|
|
2011-03-22 22:12:16 +01:00
|
|
|
/**
|
|
|
|
|
* This method constructs and returns a Future that will eventually hold the result of the execution of the supplied body
|
|
|
|
|
* The execution is performed by the specified Dispatcher.
|
|
|
|
|
*/
|
2011-12-30 13:48:31 +01:00
|
|
|
def apply[T](body: ⇒ T)(implicit executor: ExecutionContext): Future[T] = {
|
2011-12-11 00:40:52 +01:00
|
|
|
val promise = Promise[T]()
|
2011-12-30 13:48:31 +01:00
|
|
|
executor.execute(new Runnable {
|
|
|
|
|
def run =
|
|
|
|
|
promise complete {
|
|
|
|
|
try {
|
|
|
|
|
Right(body)
|
|
|
|
|
} catch {
|
2012-02-03 11:15:42 +01:00
|
|
|
case NonFatal(e) ⇒
|
|
|
|
|
executor.reportFailure(new LogEventException(Debug("Future", getClass, e.getMessage), e))
|
|
|
|
|
Left(e)
|
2011-12-30 13:48:31 +01:00
|
|
|
}
|
2011-07-26 22:23:16 -06:00
|
|
|
}
|
2011-12-30 13:48:31 +01:00
|
|
|
})
|
2011-07-26 22:23:16 -06:00
|
|
|
promise
|
|
|
|
|
}
|
2011-04-15 13:09:53 -06:00
|
|
|
|
|
|
|
|
import scala.collection.mutable.Builder
|
|
|
|
|
import scala.collection.generic.CanBuildFrom
|
|
|
|
|
|
|
|
|
|
/**
|
|
|
|
|
* Simple version of Futures.traverse. Transforms a Traversable[Future[A]] into a Future[Traversable[A]].
|
|
|
|
|
* Useful for reducing many Futures into a single Future.
|
|
|
|
|
*/
|
2011-12-30 13:48:31 +01:00
|
|
|
def sequence[A, M[_] <: Traversable[_]](in: M[Future[A]])(implicit cbf: CanBuildFrom[M[Future[A]], A, M[A]], executor: ExecutionContext): Future[M[A]] =
|
2011-12-12 22:50:08 +01:00
|
|
|
in.foldLeft(Promise.successful(cbf(in)): Future[Builder[A, M[A]]])((fr, fa) ⇒ for (r ← fr; a ← fa.asInstanceOf[Future[A]]) yield (r += a)).map(_.result)
|
2011-10-07 15:22:36 +02:00
|
|
|
|
2011-09-08 15:54:06 +02:00
|
|
|
/**
|
|
|
|
|
* Returns a Future to the result of the first future in the list that is completed
|
|
|
|
|
*/
|
2011-12-30 13:48:31 +01:00
|
|
|
def firstCompletedOf[T](futures: Traversable[Future[T]])(implicit executor: ExecutionContext): Future[T] = {
|
2011-12-11 00:40:52 +01:00
|
|
|
val futureResult = Promise[T]()
|
2011-09-08 15:54:06 +02:00
|
|
|
|
2012-02-06 12:10:37 +01:00
|
|
|
val completeFirst: Either[Throwable, T] ⇒ Unit = futureResult tryComplete _
|
2011-09-08 15:54:06 +02:00
|
|
|
futures.foreach(_ onComplete completeFirst)
|
|
|
|
|
|
|
|
|
|
futureResult
|
|
|
|
|
}
|
|
|
|
|
|
|
|
|
|
/**
|
|
|
|
|
* Returns a Future that will hold the optional result of the first Future with a result that matches the predicate
|
|
|
|
|
*/
|
2011-12-30 13:48:31 +01:00
|
|
|
def find[T](futures: Traversable[Future[T]])(predicate: T ⇒ Boolean)(implicit executor: ExecutionContext): Future[Option[T]] = {
|
2011-12-12 22:50:08 +01:00
|
|
|
if (futures.isEmpty) Promise.successful[Option[T]](None)
|
2011-09-08 15:54:06 +02:00
|
|
|
else {
|
2011-12-11 00:40:52 +01:00
|
|
|
val result = Promise[Option[T]]()
|
2011-09-08 15:54:06 +02:00
|
|
|
val ref = new AtomicInteger(futures.size)
|
2011-12-14 01:24:55 +01:00
|
|
|
val search: Either[Throwable, T] ⇒ Unit = v ⇒ try {
|
|
|
|
|
v match {
|
2012-02-06 12:10:37 +01:00
|
|
|
case Right(r) ⇒ if (predicate(r)) result tryComplete Right(Some(r))
|
2011-12-12 12:41:56 +01:00
|
|
|
case _ ⇒
|
|
|
|
|
}
|
2011-09-08 15:54:06 +02:00
|
|
|
} finally {
|
|
|
|
|
if (ref.decrementAndGet == 0)
|
2012-02-06 12:10:37 +01:00
|
|
|
result tryComplete Right(None)
|
2011-09-08 15:54:06 +02:00
|
|
|
}
|
2011-12-12 12:41:56 +01:00
|
|
|
|
2011-09-08 15:54:06 +02:00
|
|
|
futures.foreach(_ onComplete search)
|
|
|
|
|
|
|
|
|
|
result
|
|
|
|
|
}
|
|
|
|
|
}
|
|
|
|
|
|
|
|
|
|
/**
|
2011-12-27 12:02:55 +01:00
|
|
|
* A non-blocking fold over the specified futures, with the start value of the given zero.
|
2011-09-08 15:54:06 +02:00
|
|
|
* The fold is performed on the thread where the last future is completed,
|
|
|
|
|
* the result will be the first failure of any of the futures, or any failure in the actual fold,
|
|
|
|
|
* or the result of the fold.
|
|
|
|
|
* Example:
|
|
|
|
|
* <pre>
|
2011-12-27 12:02:55 +01:00
|
|
|
* val result = Await.result(Future.fold(futures)(0)(_ + _), 5 seconds)
|
2011-09-08 15:54:06 +02:00
|
|
|
* </pre>
|
|
|
|
|
*/
|
2011-12-30 13:48:31 +01:00
|
|
|
def fold[T, R](futures: Traversable[Future[T]])(zero: R)(foldFun: (R, T) ⇒ R)(implicit executor: ExecutionContext): Future[R] = {
|
2011-12-12 22:50:08 +01:00
|
|
|
if (futures.isEmpty) Promise.successful(zero)
|
2011-12-14 01:45:20 +01:00
|
|
|
else sequence(futures).map(_.foldLeft(zero)(foldFun))
|
2011-09-08 15:54:06 +02:00
|
|
|
}
|
2011-10-07 15:22:36 +02:00
|
|
|
|
2011-09-08 15:54:06 +02:00
|
|
|
/**
|
|
|
|
|
* Initiates a fold over the supplied futures where the fold-zero is the result value of the Future that's completed first
|
|
|
|
|
* Example:
|
|
|
|
|
* <pre>
|
2011-12-14 01:24:55 +01:00
|
|
|
* val result = Await.result(Futures.reduce(futures)(_ + _), 5 seconds)
|
2011-09-08 15:54:06 +02:00
|
|
|
* </pre>
|
|
|
|
|
*/
|
2012-01-18 22:01:44 +01:00
|
|
|
def reduce[T, R >: T](futures: Traversable[Future[T]])(op: (R, T) ⇒ R)(implicit executor: ExecutionContext): Future[R] = {
|
2011-12-14 01:45:20 +01:00
|
|
|
if (futures.isEmpty) Promise[R].failure(new NoSuchElementException("reduce attempted on empty collection"))
|
2012-01-18 22:01:44 +01:00
|
|
|
else sequence(futures).map(_ reduceLeft op)
|
2011-09-08 15:54:06 +02:00
|
|
|
}
|
2011-04-15 13:09:53 -06:00
|
|
|
/**
|
2011-08-06 14:03:04 +02:00
|
|
|
* Transforms a Traversable[A] into a Future[Traversable[B]] using the provided Function A ⇒ Future[B].
|
2011-04-15 13:09:53 -06:00
|
|
|
* This is useful for performing a parallel map. For example, to apply a function to all items of a list
|
|
|
|
|
* in parallel:
|
|
|
|
|
* <pre>
|
2011-12-27 12:02:55 +01:00
|
|
|
* val myFutureList = Future.traverse(myList)(x ⇒ Future(myFunc(x)))
|
2011-04-15 13:09:53 -06:00
|
|
|
* </pre>
|
|
|
|
|
*/
|
2011-12-30 13:48:31 +01:00
|
|
|
def traverse[A, B, M[_] <: Traversable[_]](in: M[A])(fn: A ⇒ Future[B])(implicit cbf: CanBuildFrom[M[A], B, M[B]], executor: ExecutionContext): Future[M[B]] =
|
2011-12-12 22:50:08 +01:00
|
|
|
in.foldLeft(Promise.successful(cbf(in)): Future[Builder[B, M[B]]]) { (fr, a) ⇒
|
2011-04-15 13:09:53 -06:00
|
|
|
val fb = fn(a.asInstanceOf[A])
|
2011-05-18 17:25:30 +02:00
|
|
|
for (r ← fr; b ← fb) yield (r += b)
|
2011-04-15 13:09:53 -06:00
|
|
|
}.map(_.result)
|
2011-10-07 15:22:36 +02:00
|
|
|
|
2011-04-23 12:57:28 -06:00
|
|
|
/**
|
|
|
|
|
* Captures a block that will be transformed into 'Continuation Passing Style' using Scala's Delimited
|
|
|
|
|
* Continuations plugin.
|
|
|
|
|
*
|
|
|
|
|
* Within the block, the result of a Future may be accessed by calling Future.apply. At that point
|
|
|
|
|
* execution is suspended with the rest of the block being stored in a continuation until the result
|
|
|
|
|
* of the Future is available. If an Exception is thrown while processing, it will be contained
|
|
|
|
|
* within the resulting Future.
|
|
|
|
|
*
|
|
|
|
|
* This allows working with Futures in an imperative style without blocking for each result.
|
|
|
|
|
*
|
2011-05-23 11:31:01 +02:00
|
|
|
* Completing a Future using 'Promise << Future' will also suspend execution until the
|
2011-04-23 12:57:28 -06:00
|
|
|
* value of the other Future is available.
|
|
|
|
|
*
|
|
|
|
|
* The Delimited Continuations compiler plugin must be enabled in order to use this method.
|
|
|
|
|
*/
|
2011-12-30 13:48:31 +01:00
|
|
|
def flow[A](body: ⇒ A @cps[Future[Any]])(implicit executor: ExecutionContext): Future[A] = {
|
2012-02-06 12:10:37 +01:00
|
|
|
val p = Promise[A]
|
2011-08-05 08:55:36 -06:00
|
|
|
dispatchTask({ () ⇒
|
2012-02-06 12:10:37 +01:00
|
|
|
(reify(body) foreachFull (p success, p failure): Future[Any]) onFailure {
|
|
|
|
|
case NonFatal(e) ⇒ p tryComplete Left(e)
|
2011-08-05 08:55:36 -06:00
|
|
|
}
|
|
|
|
|
}, true)
|
2012-02-06 12:10:37 +01:00
|
|
|
p.future
|
2011-05-02 14:44:40 -06:00
|
|
|
}
|
2011-10-07 15:22:36 +02:00
|
|
|
|
2011-10-27 20:04:32 -06:00
|
|
|
/**
|
2012-01-22 23:11:09 +01:00
|
|
|
* Signals that the current thread of execution will potentially engage
|
2012-02-20 16:37:14 +01:00
|
|
|
* an action that will take a non-trivial amount of time, perhaps by using blocking.IO or using a lot of CPU time,
|
|
|
|
|
* giving the system a chance to spawn new threads, reuse old threads or otherwise,
|
|
|
|
|
* to prevent starvation and/or unfairness.
|
2012-01-23 11:21:02 +01:00
|
|
|
*
|
2011-11-05 12:48:19 -06:00
|
|
|
* Assures that any Future tasks initiated in the current thread will be
|
|
|
|
|
* executed asynchronously, including any tasks currently queued to be
|
|
|
|
|
* executed in the current thread. This is needed if the current task may
|
|
|
|
|
* block, causing delays in executing the remaining tasks which in some
|
|
|
|
|
* cases may cause a deadlock.
|
2011-10-27 20:04:32 -06:00
|
|
|
*
|
2011-12-14 01:54:33 +01:00
|
|
|
* Note: Calling 'Await.result(future)' or 'Await.ready(future)' will automatically trigger this method.
|
2011-11-05 12:48:19 -06:00
|
|
|
*
|
|
|
|
|
* For example, in the following block of code the call to 'latch.open'
|
|
|
|
|
* might not be executed until after the call to 'latch.await', causing
|
|
|
|
|
* a deadlock. By adding 'Future.blocking()' the call to 'latch.open'
|
|
|
|
|
* will instead be dispatched separately from the current block, allowing
|
|
|
|
|
* it to be run in parallel:
|
2011-10-27 20:04:32 -06:00
|
|
|
* <pre>
|
|
|
|
|
* val latch = new StandardLatch
|
|
|
|
|
* val future = Future() map { _ ⇒
|
2011-11-05 12:48:19 -06:00
|
|
|
* Future.blocking()
|
2011-10-27 20:04:32 -06:00
|
|
|
* val nested = Future()
|
|
|
|
|
* nested foreach (_ ⇒ latch.open)
|
|
|
|
|
* latch.await
|
|
|
|
|
* }
|
|
|
|
|
* </pre>
|
|
|
|
|
*/
|
2012-02-10 08:20:36 +01:00
|
|
|
def blocking(): Unit =
|
2011-10-27 20:04:32 -06:00
|
|
|
_taskStack.get match {
|
2012-01-06 23:47:50 +01:00
|
|
|
case stack if (stack ne null) && stack.nonEmpty ⇒
|
2012-02-16 17:04:03 +01:00
|
|
|
val executionContext = _executionContext.value match {
|
2012-02-10 08:20:36 +01:00
|
|
|
case null ⇒ throw new IllegalStateException("'blocking' needs to be invoked inside a Future callback.")
|
|
|
|
|
case some ⇒ some
|
|
|
|
|
}
|
2012-01-06 23:47:50 +01:00
|
|
|
val tasks = stack.elems
|
|
|
|
|
stack.clear()
|
|
|
|
|
_taskStack.remove()
|
2012-02-10 08:20:36 +01:00
|
|
|
dispatchTask(() ⇒ _taskStack.get.elems = tasks, true)(executionContext)
|
2012-01-06 23:47:50 +01:00
|
|
|
case _ ⇒ _taskStack.remove()
|
2011-10-27 20:04:32 -06:00
|
|
|
}
|
|
|
|
|
|
2012-01-06 23:47:50 +01:00
|
|
|
private val _taskStack = new ThreadLocal[Stack[() ⇒ Unit]]()
|
2012-02-16 17:04:03 +01:00
|
|
|
private val _executionContext = new DynamicVariable[ExecutionContext](null)
|
2011-08-05 08:55:36 -06:00
|
|
|
|
2011-12-23 16:47:40 +01:00
|
|
|
/**
|
|
|
|
|
* Internal API, do not call
|
|
|
|
|
*/
|
2011-12-30 13:48:31 +01:00
|
|
|
private[akka] def dispatchTask(task: () ⇒ Unit, force: Boolean = false)(implicit executor: ExecutionContext): Unit =
|
2011-08-05 08:55:36 -06:00
|
|
|
_taskStack.get match {
|
2012-02-16 17:04:03 +01:00
|
|
|
case stack if (stack ne null) && (executor eq _executionContext.value) && !force ⇒ stack push task
|
2011-12-30 13:48:31 +01:00
|
|
|
case _ ⇒ executor.execute(
|
|
|
|
|
new Runnable {
|
|
|
|
|
def run =
|
|
|
|
|
try {
|
2012-02-16 17:04:03 +01:00
|
|
|
_executionContext.withValue(executor) {
|
|
|
|
|
val taskStack = Stack.empty[() ⇒ Unit]
|
|
|
|
|
taskStack push task
|
|
|
|
|
_taskStack set taskStack
|
|
|
|
|
|
|
|
|
|
while (taskStack.nonEmpty) {
|
|
|
|
|
val next = taskStack.pop()
|
|
|
|
|
try {
|
|
|
|
|
next.apply()
|
|
|
|
|
} catch {
|
|
|
|
|
case NonFatal(e) ⇒ executor.reportFailure(e)
|
|
|
|
|
}
|
2011-12-30 13:48:31 +01:00
|
|
|
}
|
2011-08-05 08:55:36 -06:00
|
|
|
}
|
2012-02-10 08:20:36 +01:00
|
|
|
} finally {
|
|
|
|
|
_taskStack.remove()
|
|
|
|
|
}
|
2011-12-30 13:48:31 +01:00
|
|
|
})
|
2011-08-05 08:55:36 -06:00
|
|
|
}
|
2012-02-01 14:40:12 +01:00
|
|
|
|
2011-02-28 11:47:39 -07:00
|
|
|
}
|
|
|
|
|
|
2012-01-26 14:15:25 +01:00
|
|
|
sealed trait Future[+T] extends Await.Awaitable[T] {
|
2011-03-22 22:12:16 +01:00
|
|
|
|
2012-01-30 17:01:47 +01:00
|
|
|
protected implicit def executor: ExecutionContext
|
2011-03-22 22:12:16 +01:00
|
|
|
|
2011-12-23 16:47:40 +01:00
|
|
|
protected final def resolve[X](source: Either[Throwable, X]): Either[Throwable, X] = source match {
|
|
|
|
|
case Left(t: scala.runtime.NonLocalReturnControl[_]) ⇒ Right(t.value.asInstanceOf[X])
|
2011-12-25 18:30:44 +01:00
|
|
|
case Left(t: InterruptedException) ⇒ Left(new RuntimeException("Boxed InterruptedException", t))
|
2011-12-25 16:11:01 +01:00
|
|
|
case _ ⇒ source
|
2011-12-23 16:47:40 +01:00
|
|
|
}
|
|
|
|
|
|
2012-01-19 13:50:02 +01:00
|
|
|
/**
|
2012-01-24 12:35:32 +13:00
|
|
|
* @return a new Future that will contain a tuple containing the successful result of this and that Future.
|
2012-01-19 13:50:02 +01:00
|
|
|
* If this or that fail, they will race to complete the returned Future with their failure.
|
|
|
|
|
* The returned Future will not be completed if neither this nor that are completed.
|
|
|
|
|
*/
|
|
|
|
|
def zip[U](that: Future[U]): Future[(T, U)] = {
|
|
|
|
|
val p = Promise[(T, U)]()
|
|
|
|
|
onComplete {
|
|
|
|
|
case Left(t) ⇒ p failure t
|
|
|
|
|
case Right(r) ⇒ that onSuccess { case r2 ⇒ p success ((r, r2)) }
|
|
|
|
|
}
|
2012-02-06 12:10:37 +01:00
|
|
|
that onFailure { case f ⇒ p tryComplete Left(f) }
|
2012-01-30 17:32:24 +01:00
|
|
|
p.future
|
2012-01-19 13:50:02 +01:00
|
|
|
}
|
|
|
|
|
|
2011-03-22 22:12:16 +01:00
|
|
|
/**
|
2011-04-23 12:57:28 -06:00
|
|
|
* For use only within a Future.flow block or another compatible Delimited Continuations reset block.
|
|
|
|
|
*
|
|
|
|
|
* Returns the result of this Future without blocking, by suspending execution and storing it as a
|
|
|
|
|
* continuation until the result is available.
|
2011-03-22 22:12:16 +01:00
|
|
|
*/
|
2011-12-11 00:40:52 +01:00
|
|
|
def apply(): T @cps[Future[Any]] = shift(this flatMap (_: T ⇒ Future[Any]))
|
2011-04-27 15:34:42 +02:00
|
|
|
|
2011-02-23 19:57:42 -07:00
|
|
|
/**
|
|
|
|
|
* Tests whether this Future has been completed.
|
|
|
|
|
*/
|
2012-01-19 13:50:02 +01:00
|
|
|
def isCompleted: Boolean
|
2010-10-31 19:27:55 +01:00
|
|
|
|
2011-02-23 19:57:42 -07:00
|
|
|
/**
|
|
|
|
|
* The contained value of this Future. Before this Future is completed
|
|
|
|
|
* the value will be None. After completion the value will be Some(Right(t))
|
|
|
|
|
* if it contains a valid result, or Some(Left(error)) if it contains
|
|
|
|
|
* an exception.
|
|
|
|
|
*/
|
2011-02-11 14:46:39 -07:00
|
|
|
def value: Option[Either[Throwable, T]]
|
|
|
|
|
|
2011-02-23 19:57:42 -07:00
|
|
|
/**
|
|
|
|
|
* When this Future is completed, apply the provided function to the
|
|
|
|
|
* Future. If the Future has already been completed, this will apply
|
2011-12-11 00:40:52 +01:00
|
|
|
* immediately. Multiple
|
2011-09-16 21:19:41 +02:00
|
|
|
* callbacks may be registered; there is no guarantee that they will be
|
|
|
|
|
* executed in a particular order.
|
2011-02-23 19:57:42 -07:00
|
|
|
*/
|
2012-02-06 12:10:37 +01:00
|
|
|
def onComplete[U](func: Either[Throwable, T] ⇒ U): this.type
|
2010-11-12 12:11:53 +01:00
|
|
|
|
2011-02-13 21:11:37 -07:00
|
|
|
/**
|
2011-04-23 08:11:31 +02:00
|
|
|
* When the future is completed with a valid result, apply the provided
|
2011-09-16 21:19:41 +02:00
|
|
|
* PartialFunction to the result. See `onComplete` for more details.
|
2011-03-30 21:19:26 +02:00
|
|
|
* <pre>
|
2011-12-12 20:09:26 +01:00
|
|
|
* future onSuccess {
|
2011-08-06 14:03:04 +02:00
|
|
|
* case Foo ⇒ target ! "foo"
|
|
|
|
|
* case Bar ⇒ target ! "bar"
|
2011-06-02 13:33:49 -07:00
|
|
|
* }
|
2011-03-30 21:19:26 +02:00
|
|
|
* </pre>
|
2011-02-13 21:11:37 -07:00
|
|
|
*/
|
2011-12-14 01:24:55 +01:00
|
|
|
final def onSuccess[U](pf: PartialFunction[T, U]): this.type = onComplete {
|
|
|
|
|
case Right(r) if pf isDefinedAt r ⇒ pf(r)
|
|
|
|
|
case _ ⇒
|
2011-06-02 13:33:49 -07:00
|
|
|
}
|
|
|
|
|
|
|
|
|
|
/**
|
|
|
|
|
* When the future is completed with an exception, apply the provided
|
2011-09-16 21:19:41 +02:00
|
|
|
* PartialFunction to the exception. See `onComplete` for more details.
|
2011-06-02 13:33:49 -07:00
|
|
|
* <pre>
|
2011-12-12 20:09:26 +01:00
|
|
|
* future onFailure {
|
2011-08-06 14:03:04 +02:00
|
|
|
* case NumberFormatException ⇒ target ! "wrong format"
|
2011-06-02 13:33:49 -07:00
|
|
|
* }
|
|
|
|
|
* </pre>
|
|
|
|
|
*/
|
2011-12-14 01:24:55 +01:00
|
|
|
final def onFailure[U](pf: PartialFunction[Throwable, U]): this.type = onComplete {
|
|
|
|
|
case Left(ex) if pf isDefinedAt ex ⇒ pf(ex)
|
|
|
|
|
case _ ⇒
|
2011-02-13 21:11:37 -07:00
|
|
|
}
|
|
|
|
|
|
2011-12-14 00:19:57 +01:00
|
|
|
/**
|
|
|
|
|
* Returns a failure projection of this Future
|
|
|
|
|
* If `this` becomes completed with a failure, that failure will be the success of the returned Future
|
|
|
|
|
* If `this` becomes completed with a result, then the returned future will fail with a NoSuchElementException
|
|
|
|
|
*/
|
|
|
|
|
final def failed: Future[Throwable] = {
|
|
|
|
|
val p = Promise[Throwable]()
|
2011-12-14 01:24:55 +01:00
|
|
|
this.onComplete {
|
2011-12-14 00:19:57 +01:00
|
|
|
case Left(t) ⇒ p success t
|
|
|
|
|
case Right(r) ⇒ p failure new NoSuchElementException("Future.failed not completed with a throwable. Instead completed with: " + r)
|
2011-12-14 01:24:55 +01:00
|
|
|
}
|
2012-01-30 17:32:24 +01:00
|
|
|
p.future
|
2011-12-14 00:19:57 +01:00
|
|
|
}
|
|
|
|
|
|
2011-12-11 00:40:52 +01:00
|
|
|
/**
|
2012-01-17 14:23:02 +01:00
|
|
|
* Returns a new Future that will either hold the successful value of this Future,
|
2012-01-17 15:31:59 +01:00
|
|
|
* or, it this Future fails, it will hold the result of "that" Future.
|
2011-12-11 00:40:52 +01:00
|
|
|
*/
|
2012-01-31 17:19:38 +01:00
|
|
|
def fallbackTo[U >: T](that: Future[U]): Future[U] = {
|
2012-01-17 01:18:57 +01:00
|
|
|
val p = Promise[U]()
|
2012-01-17 14:23:02 +01:00
|
|
|
onComplete {
|
|
|
|
|
case r @ Right(_) ⇒ p complete r
|
2012-01-17 15:31:59 +01:00
|
|
|
case _ ⇒ p completeWith that
|
2012-01-17 01:18:57 +01:00
|
|
|
}
|
2012-01-30 17:32:24 +01:00
|
|
|
p.future
|
2012-01-17 01:18:57 +01:00
|
|
|
}
|
2011-06-20 16:22:46 -06:00
|
|
|
|
2011-04-25 16:14:07 -06:00
|
|
|
/**
|
|
|
|
|
* Creates a new Future that will handle any matching Throwable that this
|
|
|
|
|
* Future might contain. If there is no match, or if this Future contains
|
|
|
|
|
* a valid result then the new Future will contain the same.
|
|
|
|
|
* Example:
|
|
|
|
|
* <pre>
|
2011-09-01 14:58:18 +02:00
|
|
|
* Future(6 / 0) recover { case e: ArithmeticException ⇒ 0 } // result: 0
|
|
|
|
|
* Future(6 / 0) recover { case e: NotFoundException ⇒ 0 } // result: exception
|
|
|
|
|
* Future(6 / 2) recover { case e: ArithmeticException ⇒ 0 } // result: 3
|
2011-04-25 16:14:07 -06:00
|
|
|
* </pre>
|
|
|
|
|
*/
|
2011-12-11 00:40:52 +01:00
|
|
|
final def recover[A >: T](pf: PartialFunction[Throwable, A]): Future[A] = {
|
2012-01-30 17:32:24 +01:00
|
|
|
val p = Promise[A]()
|
2011-08-06 14:03:04 +02:00
|
|
|
onComplete {
|
2012-02-06 12:10:37 +01:00
|
|
|
case Left(e) if pf isDefinedAt e ⇒ p.complete(try { Right(pf(e)) } catch { case NonFatal(x) ⇒ Left(x) })
|
2012-01-30 17:32:24 +01:00
|
|
|
case otherwise ⇒ p complete otherwise
|
2011-07-27 22:20:02 -06:00
|
|
|
}
|
2012-01-30 17:32:24 +01:00
|
|
|
p.future
|
2011-06-18 20:13:13 -06:00
|
|
|
}
|
2011-04-25 16:14:07 -06:00
|
|
|
|
2012-01-30 17:01:47 +01:00
|
|
|
/**
|
2012-01-30 17:32:24 +01:00
|
|
|
* Returns a new Future that will, in case this future fails,
|
|
|
|
|
* be completed with the resulting Future of the given PartialFunction,
|
|
|
|
|
* if the given PartialFunction matches the failure of the original Future.
|
2012-01-30 17:01:47 +01:00
|
|
|
*
|
2012-01-30 17:32:24 +01:00
|
|
|
* If the PartialFunction throws, that Throwable will be propagated to the returned Future.
|
2012-01-30 17:01:47 +01:00
|
|
|
*
|
|
|
|
|
* Example:
|
|
|
|
|
*
|
|
|
|
|
* {{{
|
|
|
|
|
* val f = Future { Int.MaxValue }
|
2012-02-03 09:25:56 +01:00
|
|
|
* Future (6 / 0) recoverWith { case e: ArithmeticException => f } // result: Int.MaxValue
|
2012-01-30 17:01:47 +01:00
|
|
|
* }}}
|
|
|
|
|
*/
|
2012-02-03 09:25:56 +01:00
|
|
|
def recoverWith[U >: T](pf: PartialFunction[Throwable, Future[U]]): Future[U] = {
|
2012-01-30 17:01:47 +01:00
|
|
|
val p = Promise[U]()
|
|
|
|
|
|
|
|
|
|
onComplete {
|
|
|
|
|
case Left(t) if pf isDefinedAt t ⇒
|
2012-02-03 13:57:28 +01:00
|
|
|
try { p completeWith pf(t) } catch { case NonFatal(t) ⇒ p complete resolve(Left(t)) }
|
2012-01-30 17:01:47 +01:00
|
|
|
case otherwise ⇒ p complete otherwise
|
|
|
|
|
}
|
|
|
|
|
|
|
|
|
|
p.future
|
|
|
|
|
}
|
|
|
|
|
|
2012-01-30 17:32:24 +01:00
|
|
|
/**
|
|
|
|
|
* Returns a new Future that will contain the completed result of this Future,
|
|
|
|
|
* and which will invoke the supplied PartialFunction when completed.
|
|
|
|
|
*
|
|
|
|
|
* This allows for establishing order of side-effects.
|
|
|
|
|
*
|
|
|
|
|
* {{{
|
|
|
|
|
* Future { 5 } andThen {
|
|
|
|
|
* case something => assert(something is awesome)
|
|
|
|
|
* } andThen {
|
|
|
|
|
* case Left(t) => handleProblem(t)
|
|
|
|
|
* case Right(v) => dealWithSuccess(v)
|
|
|
|
|
* }
|
|
|
|
|
* }}}
|
|
|
|
|
*/
|
|
|
|
|
def andThen[U](pf: PartialFunction[Either[Throwable, T], U]): Future[T] = {
|
|
|
|
|
val p = Promise[T]()
|
|
|
|
|
onComplete { case r ⇒ try if (pf isDefinedAt r) pf(r) finally p complete r }
|
|
|
|
|
p.future
|
2011-06-18 20:13:13 -06:00
|
|
|
}
|
2011-04-25 16:14:07 -06:00
|
|
|
|
2011-02-23 19:57:42 -07:00
|
|
|
/**
|
|
|
|
|
* Creates a new Future by applying a function to the successful result of
|
|
|
|
|
* this Future. If this Future is completed with an exception then the new
|
|
|
|
|
* Future will also contain this exception.
|
2011-03-30 21:19:26 +02:00
|
|
|
* Example:
|
|
|
|
|
* <pre>
|
|
|
|
|
* val future1 = for {
|
2011-06-13 13:43:21 +02:00
|
|
|
* a: Int <- actor ? "Hello" // returns 5
|
|
|
|
|
* b: String <- actor ? a // returns "10"
|
|
|
|
|
* c: String <- actor ? 7 // returns "14"
|
2011-03-30 21:19:26 +02:00
|
|
|
* } yield b + "-" + c
|
|
|
|
|
* </pre>
|
2011-02-23 19:57:42 -07:00
|
|
|
*/
|
2011-12-11 00:40:52 +01:00
|
|
|
final def map[A](f: T ⇒ A): Future[A] = {
|
|
|
|
|
val future = Promise[A]()
|
2011-08-06 14:03:04 +02:00
|
|
|
onComplete {
|
2011-12-14 01:24:55 +01:00
|
|
|
case l: Left[_, _] ⇒ future complete l.asInstanceOf[Either[Throwable, A]]
|
|
|
|
|
case Right(res) ⇒
|
|
|
|
|
future complete (try {
|
|
|
|
|
Right(f(res))
|
|
|
|
|
} catch {
|
2012-02-03 11:15:42 +01:00
|
|
|
case NonFatal(e) ⇒
|
|
|
|
|
executor.reportFailure(new LogEventException(Debug("Future", getClass, e.getMessage), e))
|
2011-12-14 01:24:55 +01:00
|
|
|
Left(e)
|
|
|
|
|
})
|
2011-07-27 22:20:02 -06:00
|
|
|
}
|
|
|
|
|
future
|
2011-06-18 20:13:13 -06:00
|
|
|
}
|
2011-02-21 17:24:51 -07:00
|
|
|
|
2011-06-13 22:36:46 +02:00
|
|
|
/**
|
|
|
|
|
* Creates a new Future[A] which is completed with this Future's result if
|
|
|
|
|
* that conforms to A's erased type or a ClassCastException otherwise.
|
2012-01-26 15:11:49 +01:00
|
|
|
*
|
|
|
|
|
* When used from Java, to create the Manifest, use:
|
2012-01-26 17:47:31 +01:00
|
|
|
* import static akka.japi.Util.manifest;
|
2012-01-26 15:11:49 +01:00
|
|
|
* future.mapTo(manifest(MyClass.class));
|
2011-06-13 22:36:46 +02:00
|
|
|
*/
|
2011-12-11 00:40:52 +01:00
|
|
|
final def mapTo[A](implicit m: Manifest[A]): Future[A] = {
|
|
|
|
|
val fa = Promise[A]()
|
2011-12-14 01:24:55 +01:00
|
|
|
onComplete {
|
|
|
|
|
case l: Left[_, _] ⇒ fa complete l.asInstanceOf[Either[Throwable, A]]
|
|
|
|
|
case Right(t) ⇒
|
|
|
|
|
fa complete (try {
|
|
|
|
|
Right(BoxedType(m.erasure).cast(t).asInstanceOf[A])
|
|
|
|
|
} catch {
|
|
|
|
|
case e: ClassCastException ⇒ Left(e)
|
|
|
|
|
})
|
2011-07-27 22:20:02 -06:00
|
|
|
}
|
2012-01-30 17:32:24 +01:00
|
|
|
fa.future
|
2011-06-18 20:13:13 -06:00
|
|
|
}
|
2011-06-13 22:36:46 +02:00
|
|
|
|
2011-02-23 19:57:42 -07:00
|
|
|
/**
|
|
|
|
|
* Creates a new Future by applying a function to the successful result of
|
|
|
|
|
* this Future, and returns the result of the function as the new Future.
|
|
|
|
|
* If this Future is completed with an exception then the new Future will
|
|
|
|
|
* also contain this exception.
|
2011-03-30 21:19:26 +02:00
|
|
|
* Example:
|
|
|
|
|
* <pre>
|
|
|
|
|
* val future1 = for {
|
2011-06-13 13:43:21 +02:00
|
|
|
* a: Int <- actor ? "Hello" // returns 5
|
|
|
|
|
* b: String <- actor ? a // returns "10"
|
|
|
|
|
* c: String <- actor ? 7 // returns "14"
|
2011-03-30 21:19:26 +02:00
|
|
|
* } yield b + "-" + c
|
|
|
|
|
* </pre>
|
2011-02-23 19:57:42 -07:00
|
|
|
*/
|
2011-12-11 00:40:52 +01:00
|
|
|
final def flatMap[A](f: T ⇒ Future[A]): Future[A] = {
|
2011-12-14 01:24:55 +01:00
|
|
|
val p = Promise[A]()
|
2011-08-06 14:03:04 +02:00
|
|
|
|
2011-07-27 22:20:02 -06:00
|
|
|
onComplete {
|
2011-12-14 01:24:55 +01:00
|
|
|
case l: Left[_, _] ⇒ p complete l.asInstanceOf[Either[Throwable, A]]
|
|
|
|
|
case Right(r) ⇒
|
|
|
|
|
try {
|
|
|
|
|
p completeWith f(r)
|
2011-08-06 14:03:04 +02:00
|
|
|
} catch {
|
2012-02-03 11:15:42 +01:00
|
|
|
case NonFatal(e) ⇒
|
|
|
|
|
executor.reportFailure(new LogEventException(Debug("Future", getClass, e.getMessage), e))
|
2011-12-14 01:24:55 +01:00
|
|
|
p complete Left(e)
|
2012-02-03 11:15:42 +01:00
|
|
|
case t ⇒
|
|
|
|
|
p complete Left(new ExecutionException(t)); throw t
|
2011-06-18 20:13:13 -06:00
|
|
|
}
|
2011-07-27 22:20:02 -06:00
|
|
|
}
|
2012-01-30 17:32:24 +01:00
|
|
|
p.future
|
2011-06-18 20:13:13 -06:00
|
|
|
}
|
2011-02-21 17:24:51 -07:00
|
|
|
|
2011-12-30 18:11:32 +01:00
|
|
|
/**
|
|
|
|
|
* Same as onSuccess { case r => f(r) } but is also used in for-comprehensions
|
|
|
|
|
*/
|
2012-02-03 09:25:56 +01:00
|
|
|
final def foreach[U](f: T ⇒ U): Unit = onComplete {
|
2011-12-14 01:24:55 +01:00
|
|
|
case Right(r) ⇒ f(r)
|
|
|
|
|
case _ ⇒
|
2011-05-26 19:33:03 +02:00
|
|
|
}
|
|
|
|
|
|
2011-12-30 18:11:32 +01:00
|
|
|
/**
|
|
|
|
|
* Used by for-comprehensions
|
|
|
|
|
*/
|
2011-12-11 00:40:52 +01:00
|
|
|
final def withFilter(p: T ⇒ Boolean) = new FutureWithFilter[T](this, p)
|
2011-05-26 19:33:03 +02:00
|
|
|
|
2011-12-11 00:40:52 +01:00
|
|
|
final class FutureWithFilter[+A](self: Future[A], p: A ⇒ Boolean) {
|
2011-05-26 19:33:03 +02:00
|
|
|
def foreach(f: A ⇒ Unit): Unit = self filter p foreach f
|
|
|
|
|
def map[B](f: A ⇒ B): Future[B] = self filter p map f
|
|
|
|
|
def flatMap[B](f: A ⇒ Future[B]): Future[B] = self filter p flatMap f
|
|
|
|
|
def withFilter(q: A ⇒ Boolean): FutureWithFilter[A] = new FutureWithFilter[A](self, x ⇒ p(x) && q(x))
|
2011-02-21 17:24:51 -07:00
|
|
|
}
|
|
|
|
|
|
2011-12-30 18:11:32 +01:00
|
|
|
/**
|
|
|
|
|
* Returns a new Future that will hold the successful result of this Future if it matches
|
|
|
|
|
* the given predicate, if it doesn't match, the resulting Future will be a failed Future
|
|
|
|
|
* with a MatchError, of if this Future fails, that failure will be propagated to the returned Future
|
|
|
|
|
*/
|
2011-12-14 01:24:55 +01:00
|
|
|
final def filter(pred: T ⇒ Boolean): Future[T] = {
|
|
|
|
|
val p = Promise[T]()
|
2011-08-06 14:03:04 +02:00
|
|
|
onComplete {
|
2011-12-14 01:24:55 +01:00
|
|
|
case l: Left[_, _] ⇒ p complete l.asInstanceOf[Either[Throwable, T]]
|
|
|
|
|
case r @ Right(res) ⇒ p complete (try {
|
|
|
|
|
if (pred(res)) r else Left(new MatchError(res))
|
|
|
|
|
} catch {
|
2012-02-03 11:15:42 +01:00
|
|
|
case NonFatal(e) ⇒
|
|
|
|
|
executor.reportFailure(new LogEventException(Debug("Future", getClass, e.getMessage), e))
|
2011-12-14 01:24:55 +01:00
|
|
|
Left(e)
|
|
|
|
|
})
|
2011-07-27 22:20:02 -06:00
|
|
|
}
|
2012-01-30 17:32:24 +01:00
|
|
|
p.future
|
2011-06-18 20:13:13 -06:00
|
|
|
}
|
2011-12-30 13:48:31 +01:00
|
|
|
|
2011-08-05 08:55:36 -06:00
|
|
|
}
|
2010-11-12 14:04:06 +01:00
|
|
|
|
2011-05-03 18:48:42 -06:00
|
|
|
object Promise {
|
2011-06-17 09:54:49 +02:00
|
|
|
/**
|
2011-12-12 17:25:34 +01:00
|
|
|
* Creates a non-completed Promise
|
2011-12-12 14:39:10 +01:00
|
|
|
*
|
|
|
|
|
* Scala API
|
2011-06-17 09:54:49 +02:00
|
|
|
*/
|
2011-12-30 13:48:31 +01:00
|
|
|
def apply[A]()(implicit executor: ExecutionContext): Promise[A] = new DefaultPromise[A]()
|
2011-12-12 17:25:34 +01:00
|
|
|
|
|
|
|
|
/**
|
|
|
|
|
* Creates an already completed Promise with the specified exception
|
|
|
|
|
*/
|
2011-12-30 13:48:31 +01:00
|
|
|
def failed[T](exception: Throwable)(implicit executor: ExecutionContext): Promise[T] = new KeptPromise[T](Left(exception))
|
2011-12-12 17:25:34 +01:00
|
|
|
|
|
|
|
|
/**
|
|
|
|
|
* Creates an already completed Promise with the specified result
|
|
|
|
|
*/
|
2011-12-30 13:48:31 +01:00
|
|
|
def successful[T](result: T)(implicit executor: ExecutionContext): Promise[T] = new KeptPromise[T](Right(result))
|
2009-05-25 14:48:43 +02:00
|
|
|
}
|
|
|
|
|
|
2011-03-18 17:37:25 +01:00
|
|
|
/**
|
2011-03-30 21:19:26 +02:00
|
|
|
* Essentially this is the Promise (or write-side) of a Future (read-side).
|
2011-03-18 17:37:25 +01:00
|
|
|
*/
|
2011-05-23 11:31:01 +02:00
|
|
|
trait Promise[T] extends Future[T] {
|
2011-12-12 17:25:34 +01:00
|
|
|
|
2011-12-14 00:19:57 +01:00
|
|
|
/**
|
|
|
|
|
* Returns the Future associated with this Promise
|
|
|
|
|
*/
|
|
|
|
|
def future: Future[T] = this
|
|
|
|
|
|
2011-03-22 22:12:16 +01:00
|
|
|
/**
|
2011-12-12 17:25:34 +01:00
|
|
|
* Completes this Promise with the specified result, if not already completed.
|
|
|
|
|
* @return whether this call completed the Promise
|
|
|
|
|
*/
|
|
|
|
|
def tryComplete(value: Either[Throwable, T]): Boolean
|
|
|
|
|
|
|
|
|
|
/**
|
|
|
|
|
* Completes this Promise with the specified result, if not already completed.
|
2012-02-06 12:10:37 +01:00
|
|
|
* @throws IllegalStateException if already completed, this is to aid in debugging of complete-races,
|
|
|
|
|
* use tryComplete to do a conditional complete.
|
2011-03-30 21:19:26 +02:00
|
|
|
* @return this
|
2011-03-22 22:12:16 +01:00
|
|
|
*/
|
2012-02-06 12:10:37 +01:00
|
|
|
final def complete(value: Either[Throwable, T]): this.type =
|
|
|
|
|
if (tryComplete(value)) this else throw new IllegalStateException("Promise already completed: " + this + " tried to complete with " + value)
|
2011-03-22 22:12:16 +01:00
|
|
|
|
|
|
|
|
/**
|
2011-12-12 17:25:34 +01:00
|
|
|
* Completes this Promise with the specified result, if not already completed.
|
2011-03-30 21:19:26 +02:00
|
|
|
* @return this
|
2011-03-22 22:12:16 +01:00
|
|
|
*/
|
2011-12-12 17:25:34 +01:00
|
|
|
final def success(result: T): this.type = complete(Right(result))
|
2011-03-22 22:12:16 +01:00
|
|
|
|
|
|
|
|
/**
|
2011-12-12 17:25:34 +01:00
|
|
|
* Completes this Promise with the specified exception, if not already completed.
|
2011-03-30 21:19:26 +02:00
|
|
|
* @return this
|
2011-03-22 22:12:16 +01:00
|
|
|
*/
|
2011-12-12 17:25:34 +01:00
|
|
|
final def failure(exception: Throwable): this.type = complete(Left(exception))
|
2011-03-22 22:12:16 +01:00
|
|
|
|
|
|
|
|
/**
|
2011-12-12 17:25:34 +01:00
|
|
|
* Completes this Promise with the specified other Future, when that Future is completed,
|
|
|
|
|
* unless this Promise has already been completed.
|
2011-03-30 21:19:26 +02:00
|
|
|
* @return this.
|
2011-03-22 22:12:16 +01:00
|
|
|
*/
|
2011-06-13 22:36:46 +02:00
|
|
|
final def completeWith(other: Future[T]): this.type = {
|
2012-02-06 12:10:37 +01:00
|
|
|
other onComplete { tryComplete(_) }
|
2011-03-16 21:17:38 +01:00
|
|
|
this
|
2010-11-12 12:52:08 +01:00
|
|
|
}
|
2011-03-22 22:12:16 +01:00
|
|
|
|
2011-05-18 17:25:30 +02:00
|
|
|
final def <<(value: T): Future[T] @cps[Future[Any]] = shift { cont: (Future[T] ⇒ Future[Any]) ⇒ cont(complete(Right(value))) }
|
2011-03-22 22:12:16 +01:00
|
|
|
|
2011-05-18 17:25:30 +02:00
|
|
|
final def <<(other: Future[T]): Future[T] @cps[Future[Any]] = shift { cont: (Future[T] ⇒ Future[Any]) ⇒
|
2011-12-11 00:40:52 +01:00
|
|
|
val fr = Promise[Any]()
|
2011-12-14 01:24:55 +01:00
|
|
|
val thisPromise = this
|
|
|
|
|
thisPromise completeWith other onComplete { v ⇒
|
2011-04-23 09:14:20 -06:00
|
|
|
try {
|
2011-12-14 01:24:55 +01:00
|
|
|
fr completeWith cont(thisPromise)
|
2011-04-23 09:14:20 -06:00
|
|
|
} catch {
|
2012-02-03 11:15:42 +01:00
|
|
|
case NonFatal(e) ⇒
|
|
|
|
|
executor.reportFailure(new LogEventException(Debug("Future", getClass, e.getMessage), e))
|
2011-12-12 17:25:34 +01:00
|
|
|
fr failure e
|
2011-04-23 09:14:20 -06:00
|
|
|
}
|
|
|
|
|
}
|
|
|
|
|
fr
|
2011-04-21 15:12:47 -06:00
|
|
|
}
|
2011-03-22 22:12:16 +01:00
|
|
|
|
2011-05-19 14:43:58 -06:00
|
|
|
final def <<(stream: PromiseStreamOut[T]): Future[T] @cps[Future[Any]] = shift { cont: (Future[T] ⇒ Future[Any]) ⇒
|
2011-12-11 00:40:52 +01:00
|
|
|
val fr = Promise[Any]()
|
2011-12-14 01:24:55 +01:00
|
|
|
val f = stream.dequeue(this)
|
|
|
|
|
f.onComplete { _ ⇒
|
2011-05-19 14:43:58 -06:00
|
|
|
try {
|
|
|
|
|
fr completeWith cont(f)
|
|
|
|
|
} catch {
|
2012-02-03 11:15:42 +01:00
|
|
|
case NonFatal(e) ⇒
|
|
|
|
|
executor.reportFailure(new LogEventException(Debug("Future", getClass, e.getMessage), e))
|
2011-12-12 17:25:34 +01:00
|
|
|
fr failure e
|
2011-05-19 14:43:58 -06:00
|
|
|
}
|
|
|
|
|
}
|
|
|
|
|
fr
|
|
|
|
|
}
|
2009-05-25 14:48:43 +02:00
|
|
|
}
|
|
|
|
|
|
2011-08-05 22:27:16 +02:00
|
|
|
//Companion object to FState, just to provide a cheap, immutable default entry
|
2011-10-29 00:36:42 +02:00
|
|
|
private[dispatch] object DefaultPromise {
|
2012-01-19 13:50:02 +01:00
|
|
|
def EmptyPending[T](): List[T] = Nil
|
2011-10-12 16:29:33 +02:00
|
|
|
}
|
2011-08-05 22:27:16 +02:00
|
|
|
|
2011-02-28 10:17:44 -07:00
|
|
|
/**
|
2011-03-30 21:19:26 +02:00
|
|
|
* The default concrete Future implementation.
|
2011-02-28 10:17:44 -07:00
|
|
|
*/
|
2011-12-30 13:48:31 +01:00
|
|
|
class DefaultPromise[T](implicit val executor: ExecutionContext) extends AbstractPromise with Promise[T] {
|
2011-06-17 06:51:40 -06:00
|
|
|
self ⇒
|
2010-10-31 19:27:55 +01:00
|
|
|
|
2011-12-11 01:29:46 +01:00
|
|
|
protected final def tryAwait(atMost: Duration): Boolean = {
|
2011-12-11 00:40:52 +01:00
|
|
|
Future.blocking
|
2011-06-18 23:23:47 -06:00
|
|
|
|
2011-12-11 00:40:52 +01:00
|
|
|
@tailrec
|
|
|
|
|
def awaitUnsafe(waitTimeNanos: Long): Boolean = {
|
2012-01-19 13:50:02 +01:00
|
|
|
if (!isCompleted && waitTimeNanos > 0) {
|
2011-12-11 00:40:52 +01:00
|
|
|
val ms = NANOSECONDS.toMillis(waitTimeNanos)
|
|
|
|
|
val ns = (waitTimeNanos % 1000000l).toInt //As per object.wait spec
|
2011-12-11 01:29:46 +01:00
|
|
|
val start = System.nanoTime()
|
2012-01-19 13:50:02 +01:00
|
|
|
try { synchronized { if (!isCompleted) wait(ms, ns) } } catch { case e: InterruptedException ⇒ }
|
2011-12-11 00:40:52 +01:00
|
|
|
|
2011-12-11 01:29:46 +01:00
|
|
|
awaitUnsafe(waitTimeNanos - (System.nanoTime() - start))
|
2012-01-19 13:50:02 +01:00
|
|
|
} else isCompleted
|
2011-02-07 18:59:49 +01:00
|
|
|
}
|
2011-12-11 01:29:46 +01:00
|
|
|
awaitUnsafe(if (atMost.isFinite) atMost.toNanos else Long.MaxValue)
|
|
|
|
|
}
|
2011-10-27 20:04:32 -06:00
|
|
|
|
2012-02-16 12:31:49 +01:00
|
|
|
@throws(classOf[TimeoutException])
|
2011-12-12 22:50:08 +01:00
|
|
|
def ready(atMost: Duration)(implicit permit: CanAwait): this.type =
|
2012-01-19 13:50:02 +01:00
|
|
|
if (isCompleted || tryAwait(atMost)) this
|
2011-12-11 01:29:46 +01:00
|
|
|
else throw new TimeoutException("Futures timed out after [" + atMost.toMillis + "] milliseconds")
|
2011-08-06 20:05:43 +02:00
|
|
|
|
2012-02-16 12:31:49 +01:00
|
|
|
@throws(classOf[Exception])
|
2011-12-12 22:50:08 +01:00
|
|
|
def result(atMost: Duration)(implicit permit: CanAwait): T =
|
|
|
|
|
ready(atMost).value.get match {
|
2012-02-10 14:13:40 +01:00
|
|
|
case Left(e: AskTimeoutException) ⇒ throw new AskTimeoutException(e.getMessage, e) // to get meaningful stack trace
|
2012-02-09 22:58:28 +01:00
|
|
|
case Left(e) ⇒ throw e
|
|
|
|
|
case Right(r) ⇒ r
|
2011-12-11 14:06:30 +01:00
|
|
|
}
|
2009-05-25 14:48:43 +02:00
|
|
|
|
2012-01-19 13:50:02 +01:00
|
|
|
def value: Option[Either[Throwable, T]] = getState match {
|
|
|
|
|
case _: List[_] ⇒ None
|
|
|
|
|
case c: Either[_, _] ⇒ Some(c.asInstanceOf[Either[Throwable, T]])
|
|
|
|
|
}
|
|
|
|
|
|
|
|
|
|
def isCompleted(): Boolean = getState match {
|
|
|
|
|
case _: Either[_, _] ⇒ true
|
|
|
|
|
case _ ⇒ false
|
|
|
|
|
}
|
2011-10-04 21:17:01 +02:00
|
|
|
|
|
|
|
|
@inline
|
2012-01-19 13:50:02 +01:00
|
|
|
private[this] final def updater = AbstractPromise.updater.asInstanceOf[AtomicReferenceFieldUpdater[AbstractPromise, AnyRef]]
|
2011-10-04 21:17:01 +02:00
|
|
|
|
|
|
|
|
@inline
|
2012-01-19 13:50:02 +01:00
|
|
|
protected final def updateState(oldState: AnyRef, newState: AnyRef): Boolean = updater.compareAndSet(this, oldState, newState)
|
2011-10-12 16:29:33 +02:00
|
|
|
|
2011-12-11 00:40:52 +01:00
|
|
|
@inline
|
2012-01-19 13:50:02 +01:00
|
|
|
protected final def getState: AnyRef = updater.get(this)
|
2010-11-12 12:11:53 +01:00
|
|
|
|
2011-12-12 17:25:34 +01:00
|
|
|
def tryComplete(value: Either[Throwable, T]): Boolean = {
|
2011-12-14 01:24:55 +01:00
|
|
|
val callbacks: List[Either[Throwable, T] ⇒ Unit] = {
|
2011-07-28 09:57:45 -06:00
|
|
|
try {
|
2011-08-05 22:27:16 +02:00
|
|
|
@tailrec
|
2011-12-23 16:47:40 +01:00
|
|
|
def tryComplete(v: Either[Throwable, T]): List[Either[Throwable, T] ⇒ Unit] = {
|
|
|
|
|
getState match {
|
2012-01-19 13:50:02 +01:00
|
|
|
case raw: List[_] ⇒
|
|
|
|
|
val cur = raw.asInstanceOf[List[Either[Throwable, T] ⇒ Unit]]
|
|
|
|
|
if (updateState(cur, v)) cur else tryComplete(v)
|
2011-12-12 17:25:34 +01:00
|
|
|
case _ ⇒ null
|
2011-07-28 09:57:45 -06:00
|
|
|
}
|
2011-04-29 13:22:39 +02:00
|
|
|
}
|
2011-12-23 16:47:40 +01:00
|
|
|
tryComplete(resolve(value))
|
2011-07-28 09:57:45 -06:00
|
|
|
} finally {
|
2011-10-04 21:17:01 +02:00
|
|
|
synchronized { notifyAll() } //Notify any evil blockers
|
2011-04-28 20:53:45 -06:00
|
|
|
}
|
|
|
|
|
}
|
2011-02-28 10:17:44 -07:00
|
|
|
|
2011-12-12 17:25:34 +01:00
|
|
|
callbacks match {
|
|
|
|
|
case null ⇒ false
|
|
|
|
|
case cs if cs.isEmpty ⇒ true
|
2011-12-14 01:45:20 +01:00
|
|
|
case cs ⇒ Future.dispatchTask(() ⇒ cs.foreach(f ⇒ notifyCompleted(f, value))); true
|
2011-12-12 17:25:34 +01:00
|
|
|
}
|
2010-11-12 12:11:53 +01:00
|
|
|
}
|
|
|
|
|
|
2012-02-06 12:10:37 +01:00
|
|
|
def onComplete[U](func: Either[Throwable, T] ⇒ U): this.type = {
|
2011-08-05 22:27:16 +02:00
|
|
|
@tailrec //Returns whether the future has already been completed or not
|
2012-01-19 13:50:02 +01:00
|
|
|
def tryAddCallback(): Either[Throwable, T] = {
|
2011-10-04 21:17:01 +02:00
|
|
|
val cur = getState
|
2011-10-12 16:29:33 +02:00
|
|
|
cur match {
|
2012-01-19 13:50:02 +01:00
|
|
|
case r: Either[_, _] ⇒ r.asInstanceOf[Either[Throwable, T]]
|
|
|
|
|
case listeners: List[_] ⇒ if (updateState(listeners, func :: listeners)) null else tryAddCallback()
|
2011-10-12 16:29:33 +02:00
|
|
|
}
|
2011-02-28 10:17:44 -07:00
|
|
|
}
|
|
|
|
|
|
2012-01-19 13:50:02 +01:00
|
|
|
tryAddCallback() match {
|
|
|
|
|
case null ⇒ this
|
|
|
|
|
case completed ⇒
|
|
|
|
|
Future.dispatchTask(() ⇒ notifyCompleted(func, completed))
|
|
|
|
|
this
|
2011-12-14 01:45:20 +01:00
|
|
|
}
|
2009-05-25 14:48:43 +02:00
|
|
|
}
|
|
|
|
|
|
2012-02-06 12:10:37 +01:00
|
|
|
private final def notifyCompleted[U](func: Either[Throwable, T] ⇒ U, result: Either[Throwable, T]): Unit =
|
|
|
|
|
try func(result) catch { case NonFatal(e) ⇒ executor reportFailure e }
|
2009-05-25 14:48:43 +02:00
|
|
|
}
|
2011-03-18 17:26:53 +01:00
|
|
|
|
2011-03-18 17:37:25 +01:00
|
|
|
/**
|
|
|
|
|
* An already completed Future is seeded with it's result at creation, is useful for when you are participating in
|
|
|
|
|
* a Future-composition but you already have a value to contribute.
|
|
|
|
|
*/
|
2011-12-30 13:48:31 +01:00
|
|
|
final class KeptPromise[T](suppliedValue: Either[Throwable, T])(implicit val executor: ExecutionContext) extends Promise[T] {
|
2011-12-23 16:47:40 +01:00
|
|
|
val value = Some(resolve(suppliedValue))
|
2011-03-18 17:26:53 +01:00
|
|
|
|
2012-01-06 23:16:59 +01:00
|
|
|
def tryComplete(value: Either[Throwable, T]): Boolean = false
|
2012-02-06 12:10:37 +01:00
|
|
|
def onComplete[U](func: Either[Throwable, T] ⇒ U): this.type = {
|
2011-12-14 01:24:55 +01:00
|
|
|
val completedAs = value.get
|
|
|
|
|
Future dispatchTask (() ⇒ func(completedAs))
|
2011-07-28 10:10:41 -06:00
|
|
|
this
|
|
|
|
|
}
|
2012-01-19 13:50:02 +01:00
|
|
|
def isCompleted(): Boolean = true
|
2011-12-12 22:50:08 +01:00
|
|
|
def ready(atMost: Duration)(implicit permit: CanAwait): this.type = this
|
|
|
|
|
def result(atMost: Duration)(implicit permit: CanAwait): T = value.get match {
|
2011-12-11 14:06:30 +01:00
|
|
|
case Left(e) ⇒ throw e
|
|
|
|
|
case Right(r) ⇒ r
|
|
|
|
|
}
|
2011-03-18 17:26:53 +01:00
|
|
|
}
|
2012-01-26 15:11:49 +01:00
|
|
|
|
|
|
|
|
/**
|
|
|
|
|
* This class contains bridge classes between Scala and Java.
|
|
|
|
|
* Internal use only.
|
|
|
|
|
*/
|
2012-01-26 14:15:25 +01:00
|
|
|
object japi {
|
|
|
|
|
@deprecated("Do not use this directly, use subclasses of this", "2.0")
|
2012-02-24 15:03:36 +01:00
|
|
|
class CallbackBridge[-T] extends PartialFunction[T, BoxedUnit] {
|
2012-01-26 14:15:25 +01:00
|
|
|
override final def isDefinedAt(t: T): Boolean = true
|
2012-02-24 15:03:36 +01:00
|
|
|
override final def apply(t: T): BoxedUnit = {
|
|
|
|
|
internal(t)
|
|
|
|
|
BoxedUnit.UNIT
|
|
|
|
|
}
|
2012-01-26 14:15:25 +01:00
|
|
|
protected def internal(result: T): Unit = ()
|
|
|
|
|
}
|
|
|
|
|
|
|
|
|
|
@deprecated("Do not use this directly, use 'Recover'", "2.0")
|
|
|
|
|
class RecoverBridge[+T] extends PartialFunction[Throwable, T] {
|
|
|
|
|
override final def isDefinedAt(t: Throwable): Boolean = true
|
|
|
|
|
override final def apply(t: Throwable): T = internal(t)
|
|
|
|
|
protected def internal(result: Throwable): T = null.asInstanceOf[T]
|
|
|
|
|
}
|
|
|
|
|
|
|
|
|
|
@deprecated("Do not use this directly, use subclasses of this", "2.0")
|
|
|
|
|
class BooleanFunctionBridge[-T] extends scala.Function1[T, Boolean] {
|
|
|
|
|
override final def apply(t: T): Boolean = internal(t)
|
|
|
|
|
protected def internal(result: T): Boolean = false
|
|
|
|
|
}
|
|
|
|
|
|
|
|
|
|
@deprecated("Do not use this directly, use subclasses of this", "2.0")
|
2012-02-24 15:03:36 +01:00
|
|
|
class UnitFunctionBridge[-T] extends (T ⇒ BoxedUnit) {
|
|
|
|
|
override final def apply(t: T): BoxedUnit = {
|
|
|
|
|
internal(t)
|
|
|
|
|
BoxedUnit.UNIT
|
|
|
|
|
}
|
2012-01-26 14:15:25 +01:00
|
|
|
protected def internal(result: T): Unit = ()
|
|
|
|
|
}
|
|
|
|
|
}
|
|
|
|
|
|
2012-01-26 15:11:49 +01:00
|
|
|
/**
|
|
|
|
|
* Callback for when a Future is completed successfully
|
|
|
|
|
* SAM (Single Abstract Method) class
|
|
|
|
|
*
|
|
|
|
|
* Java API
|
|
|
|
|
*/
|
2012-01-26 14:15:25 +01:00
|
|
|
abstract class OnSuccess[-T] extends japi.CallbackBridge[T] {
|
|
|
|
|
protected final override def internal(result: T) = onSuccess(result)
|
2012-01-26 15:11:49 +01:00
|
|
|
|
|
|
|
|
/**
|
|
|
|
|
* This method will be invoked once when/if a Future that this callback is registered on
|
|
|
|
|
* becomes successfully completed
|
|
|
|
|
*/
|
2012-01-26 14:15:25 +01:00
|
|
|
def onSuccess(result: T): Unit
|
|
|
|
|
}
|
|
|
|
|
|
2012-01-26 15:11:49 +01:00
|
|
|
/**
|
|
|
|
|
* Callback for when a Future is completed with a failure
|
|
|
|
|
* SAM (Single Abstract Method) class
|
|
|
|
|
*
|
|
|
|
|
* Java API
|
|
|
|
|
*/
|
2012-01-26 14:15:25 +01:00
|
|
|
abstract class OnFailure extends japi.CallbackBridge[Throwable] {
|
|
|
|
|
protected final override def internal(failure: Throwable) = onFailure(failure)
|
2012-01-26 15:11:49 +01:00
|
|
|
|
|
|
|
|
/**
|
|
|
|
|
* This method will be invoked once when/if a Future that this callback is registered on
|
|
|
|
|
* becomes completed with a failure
|
|
|
|
|
*/
|
2012-01-26 14:15:25 +01:00
|
|
|
def onFailure(failure: Throwable): Unit
|
|
|
|
|
}
|
|
|
|
|
|
2012-01-26 15:11:49 +01:00
|
|
|
/**
|
|
|
|
|
* Callback for when a Future is completed with either failure or a success
|
|
|
|
|
* SAM (Single Abstract Method) class
|
|
|
|
|
*
|
|
|
|
|
* Java API
|
|
|
|
|
*/
|
2012-01-26 14:15:25 +01:00
|
|
|
abstract class OnComplete[-T] extends japi.CallbackBridge[Either[Throwable, T]] {
|
|
|
|
|
protected final override def internal(value: Either[Throwable, T]): Unit = value match {
|
|
|
|
|
case Left(t) ⇒ onComplete(t, null.asInstanceOf[T])
|
|
|
|
|
case Right(r) ⇒ onComplete(null, r)
|
|
|
|
|
}
|
2012-01-26 15:11:49 +01:00
|
|
|
|
|
|
|
|
/**
|
|
|
|
|
* This method will be invoked once when/if a Future that this callback is registered on
|
|
|
|
|
* becomes completed with a failure or a success.
|
|
|
|
|
* In the case of success then "failure" will be null, and in the case of failure the "success" will be null.
|
|
|
|
|
*/
|
2012-01-26 14:15:25 +01:00
|
|
|
def onComplete(failure: Throwable, success: T): Unit
|
|
|
|
|
}
|
|
|
|
|
|
2012-01-26 15:11:49 +01:00
|
|
|
/**
|
|
|
|
|
* Callback for the Future.recover operation that conditionally turns failures into successes.
|
|
|
|
|
*
|
|
|
|
|
* SAM (Single Abstract Method) class
|
|
|
|
|
*
|
|
|
|
|
* Java API
|
|
|
|
|
*/
|
2012-01-26 14:15:25 +01:00
|
|
|
abstract class Recover[+T] extends japi.RecoverBridge[T] {
|
|
|
|
|
protected final override def internal(result: Throwable): T = recover(result)
|
2012-01-26 15:11:49 +01:00
|
|
|
|
|
|
|
|
/**
|
|
|
|
|
* This method will be invoked once when/if the Future this recover callback is registered on
|
|
|
|
|
* becomes completed with a failure.
|
|
|
|
|
*
|
2012-02-06 16:59:09 +01:00
|
|
|
* @return a successful value for the passed in failure
|
2012-01-26 15:11:49 +01:00
|
|
|
* @throws the passed in failure to propagate it.
|
|
|
|
|
*
|
|
|
|
|
* Java API
|
|
|
|
|
*/
|
|
|
|
|
@throws(classOf[Throwable])
|
2012-01-26 14:15:25 +01:00
|
|
|
def recover(failure: Throwable): T
|
|
|
|
|
}
|
|
|
|
|
|
2012-01-26 15:11:49 +01:00
|
|
|
/**
|
|
|
|
|
* Callback for the Future.filter operation that creates a new Future which will
|
|
|
|
|
* conditionally contain the success of another Future.
|
|
|
|
|
*
|
|
|
|
|
* SAM (Single Abstract Method) class
|
|
|
|
|
* Java API
|
|
|
|
|
*/
|
2012-01-26 14:15:25 +01:00
|
|
|
abstract class Filter[-T] extends japi.BooleanFunctionBridge[T] {
|
|
|
|
|
override final def internal(t: T): Boolean = filter(t)
|
2012-01-26 15:11:49 +01:00
|
|
|
|
|
|
|
|
/**
|
|
|
|
|
* This method will be invoked once when/if a Future that this callback is registered on
|
|
|
|
|
* becomes completed with a success.
|
|
|
|
|
*
|
2012-02-06 16:59:09 +01:00
|
|
|
* @return true if the successful value should be propagated to the new Future or not
|
2012-01-26 15:11:49 +01:00
|
|
|
*/
|
2012-01-26 14:15:25 +01:00
|
|
|
def filter(result: T): Boolean
|
|
|
|
|
}
|
|
|
|
|
|
2012-01-26 15:11:49 +01:00
|
|
|
/**
|
|
|
|
|
* Callback for the Future.foreach operation that will be invoked if the Future that this callback
|
|
|
|
|
* is registered on becomes completed with a success. This method is essentially the same operation
|
|
|
|
|
* as onSuccess.
|
|
|
|
|
*
|
|
|
|
|
* SAM (Single Abstract Method) class
|
|
|
|
|
* Java API
|
|
|
|
|
*/
|
2012-01-26 14:15:25 +01:00
|
|
|
abstract class Foreach[-T] extends japi.UnitFunctionBridge[T] {
|
|
|
|
|
override final def internal(t: T): Unit = each(t)
|
2012-01-26 15:11:49 +01:00
|
|
|
|
|
|
|
|
/**
|
|
|
|
|
* This method will be invoked once when/if a Future that this callback is registered on
|
|
|
|
|
* becomes successfully completed
|
|
|
|
|
*/
|
2012-01-26 14:15:25 +01:00
|
|
|
def each(result: T): Unit
|
|
|
|
|
}
|
|
|
|
|
|
2012-01-26 15:11:49 +01:00
|
|
|
/**
|
|
|
|
|
* Callback for the Future.map and Future.flatMap operations that will be invoked
|
|
|
|
|
* if the Future that this callback is registered on becomes completed with a success.
|
|
|
|
|
* This callback is the equivalent of an akka.japi.Function
|
|
|
|
|
*
|
|
|
|
|
* SAM (Single Abstract Method) class
|
|
|
|
|
*
|
|
|
|
|
* Java API
|
|
|
|
|
*/
|
2012-01-26 14:16:24 +01:00
|
|
|
abstract class Mapper[-T, +R] extends scala.runtime.AbstractFunction1[T, R]
|