2009-05-25 14:48:43 +02:00
|
|
|
/**
|
2011-07-14 16:03:08 +02:00
|
|
|
* Copyright (C) 2009-2011 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
|
|
|
|
2010-10-26 12:49:25 +02:00
|
|
|
import akka.AkkaException
|
2011-03-23 15:12:09 +01:00
|
|
|
import akka.event.EventHandler
|
2011-06-28 15:20:31 -06:00
|
|
|
import akka.actor.{ Actor, Channel, ForwardableChannel, NullChannel, UntypedChannel, ActorRef, Scheduler, Timeout }
|
2011-06-13 22:36:46 +02:00
|
|
|
import akka.util.{ Duration, BoxedType }
|
2011-05-18 17:25:30 +02:00
|
|
|
import akka.japi.{ Procedure, Function ⇒ JFunc }
|
2010-10-31 19:27:55 +01:00
|
|
|
|
2011-04-21 15:12:47 -06:00
|
|
|
import scala.util.continuations._
|
|
|
|
|
|
2010-08-19 07:01:09 +02:00
|
|
|
import java.util.concurrent.locks.ReentrantLock
|
2011-05-18 17:25:30 +02:00
|
|
|
import java.util.concurrent.{ ConcurrentLinkedQueue, TimeUnit, Callable }
|
|
|
|
|
import java.util.concurrent.TimeUnit.{ NANOSECONDS ⇒ NANOS, MILLISECONDS ⇒ MILLIS }
|
|
|
|
|
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
|
2011-07-10 20:18:43 +02:00
|
|
|
import akka.util.{ Switch, Duration, BoxedType }
|
|
|
|
|
import java.util.concurrent.atomic.{ AtomicLong, AtomicBoolean }
|
2009-05-25 14:48:43 +02:00
|
|
|
|
2011-04-29 17:15:00 +02:00
|
|
|
class FutureTimeoutException(message: String, cause: Throwable = null) extends AkkaException(message, cause)
|
2009-06-10 20:04:33 +02:00
|
|
|
|
2010-02-23 10:05:47 +01:00
|
|
|
object Futures {
|
|
|
|
|
|
2011-03-22 17:20:35 +01:00
|
|
|
/**
|
|
|
|
|
* Java API, equivalent to Future.apply
|
|
|
|
|
*/
|
2011-02-28 16:29:03 -07:00
|
|
|
def future[T](body: Callable[T]): Future[T] =
|
|
|
|
|
Future(body.call)
|
|
|
|
|
|
2011-06-18 23:23:47 -06:00
|
|
|
/**
|
|
|
|
|
* Java API, equivalent to Future.apply
|
|
|
|
|
*/
|
|
|
|
|
def future[T](body: Callable[T], timeout: Timeout): Future[T] =
|
|
|
|
|
Future(body.call, timeout)
|
|
|
|
|
|
2011-03-22 17:20:35 +01:00
|
|
|
/**
|
|
|
|
|
* Java API, equivalent to Future.apply
|
|
|
|
|
*/
|
2011-02-28 16:29:03 -07:00
|
|
|
def future[T](body: Callable[T], timeout: Long): Future[T] =
|
|
|
|
|
Future(body.call, timeout)
|
|
|
|
|
|
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-03-22 17:20:35 +01:00
|
|
|
def future[T](body: Callable[T], dispatcher: MessageDispatcher): Future[T] =
|
|
|
|
|
Future(body.call)(dispatcher)
|
2010-03-01 22:03:17 +01:00
|
|
|
|
2011-06-18 23:23:47 -06:00
|
|
|
/**
|
|
|
|
|
* Java API, equivalent to Future.apply
|
|
|
|
|
*/
|
|
|
|
|
def future[T](body: Callable[T], timeout: Timeout, dispatcher: MessageDispatcher): Future[T] =
|
2011-06-25 14:20:54 -06:00
|
|
|
Future(body.call)(dispatcher, timeout)
|
2011-06-18 23:23:47 -06:00
|
|
|
|
2010-10-26 16:40:09 +02:00
|
|
|
/**
|
2011-03-22 17:20:35 +01:00
|
|
|
* Java API, equivalent to Future.apply
|
2010-10-26 16:40:09 +02:00
|
|
|
*/
|
2011-03-22 17:20:35 +01:00
|
|
|
def future[T](body: Callable[T], timeout: Long, dispatcher: MessageDispatcher): Future[T] =
|
2011-06-25 14:20:54 -06:00
|
|
|
Future(body.call)(dispatcher, timeout)
|
2010-11-12 12:54:48 +01:00
|
|
|
|
|
|
|
|
/**
|
|
|
|
|
* Returns a Future to the result of the first future in the list that is completed
|
|
|
|
|
*/
|
2011-06-19 12:05:27 -06:00
|
|
|
def firstCompletedOf[T](futures: Iterable[Future[T]], timeout: Timeout = Timeout.never): Future[T] = {
|
2011-05-23 11:31:01 +02:00
|
|
|
val futureResult = new DefaultPromise[T](timeout)
|
2011-02-08 16:14:40 +01:00
|
|
|
|
2011-05-18 17:25:30 +02:00
|
|
|
val completeFirst: Future[T] ⇒ Unit = _.value.foreach(futureResult complete _)
|
|
|
|
|
for (f ← futures) f onComplete completeFirst
|
2011-02-08 16:14:40 +01:00
|
|
|
|
2010-11-12 12:54:48 +01:00
|
|
|
futureResult
|
2010-02-23 10:05:47 +01: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-06-18 23:23:47 -06:00
|
|
|
def firstCompletedOf[T <: AnyRef](futures: java.lang.Iterable[Future[T]], timeout: Timeout): Future[T] =
|
2011-05-18 17:25:30 +02:00
|
|
|
firstCompletedOf(scala.collection.JavaConversions.iterableAsScalaIterable(futures), timeout)
|
2011-03-22 22:12:16 +01:00
|
|
|
|
2011-01-24 13:10:52 +01:00
|
|
|
/**
|
|
|
|
|
* A non-blocking fold over the specified futures.
|
|
|
|
|
* 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-03-30 21:19:26 +02:00
|
|
|
* Example:
|
|
|
|
|
* <pre>
|
|
|
|
|
* val result = Futures.fold(0)(futures)(_ + _).await.result
|
|
|
|
|
* </pre>
|
2011-01-24 13:10:52 +01:00
|
|
|
*/
|
2011-06-18 23:23:47 -06:00
|
|
|
def fold[T, R](zero: R, timeout: Timeout = Timeout.default)(futures: Iterable[Future[T]])(foldFun: (R, T) ⇒ R): Future[R] = {
|
2011-05-18 17:25:30 +02:00
|
|
|
if (futures.isEmpty) {
|
2011-05-23 11:31:01 +02:00
|
|
|
new KeptPromise[R](Right(zero))
|
2011-02-08 16:14:40 +01:00
|
|
|
} else {
|
2011-05-23 11:31:01 +02:00
|
|
|
val result = new DefaultPromise[R](timeout)
|
2011-02-08 16:14:40 +01:00
|
|
|
val results = new ConcurrentLinkedQueue[T]()
|
2011-07-10 20:18:43 +02:00
|
|
|
val done = new Switch(false)
|
2011-03-18 01:21:26 +01:00
|
|
|
val allDone = futures.size
|
2011-02-08 16:14:40 +01:00
|
|
|
|
2011-07-10 20:18:43 +02:00
|
|
|
val aggregate: Future[T] ⇒ Unit = f ⇒ if (done.isOff && !result.isCompleted) { //TODO: This is an optimization, is it premature?
|
2011-03-18 01:21:26 +01:00
|
|
|
f.value.get match {
|
2011-07-19 19:28:46 +02:00
|
|
|
case Right(value) ⇒
|
|
|
|
|
val added = results add value
|
2011-07-10 20:18:43 +02:00
|
|
|
if (added && results.size == allDone) { //Only one thread can get here
|
|
|
|
|
if (done.switchOn) {
|
|
|
|
|
try {
|
|
|
|
|
val i = results.iterator
|
|
|
|
|
var currentValue = zero
|
|
|
|
|
while (i.hasNext) { currentValue = foldFun(currentValue, i.next) }
|
|
|
|
|
result completeWithResult currentValue
|
|
|
|
|
} catch {
|
|
|
|
|
case e: Exception ⇒
|
|
|
|
|
EventHandler.error(e, this, e.getMessage)
|
|
|
|
|
result completeWithException e
|
2011-07-26 18:33:59 +12:00
|
|
|
} finally {
|
2011-07-10 20:18:43 +02:00
|
|
|
results.clear
|
|
|
|
|
}
|
2011-03-18 01:21:26 +01:00
|
|
|
}
|
2011-02-08 16:14:40 +01:00
|
|
|
}
|
2011-07-19 19:28:46 +02:00
|
|
|
case Left(exception) ⇒
|
2011-07-10 20:18:43 +02:00
|
|
|
if (done.switchOn) {
|
2011-07-19 19:28:46 +02:00
|
|
|
result completeWithException exception
|
2011-07-10 20:18:43 +02:00
|
|
|
results.clear
|
|
|
|
|
}
|
2011-01-24 12:14:50 +01:00
|
|
|
}
|
|
|
|
|
}
|
|
|
|
|
|
2011-01-24 17:42:56 +01:00
|
|
|
futures foreach { _ onComplete aggregate }
|
2011-02-08 16:14:40 +01:00
|
|
|
result
|
|
|
|
|
}
|
2011-01-24 12:14:50 +01:00
|
|
|
}
|
2011-01-24 16:37:08 +01:00
|
|
|
|
2011-03-22 22:12:16 +01:00
|
|
|
/**
|
|
|
|
|
* Java API
|
|
|
|
|
* A non-blocking fold over the specified futures.
|
|
|
|
|
* 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-06-18 23:23:47 -06:00
|
|
|
def fold[T <: AnyRef, R <: AnyRef](zero: R, timeout: Timeout, futures: java.lang.Iterable[Future[T]], fun: akka.japi.Function2[R, T, R]): Future[R] =
|
2011-05-18 17:25:30 +02:00
|
|
|
fold(zero, timeout)(scala.collection.JavaConversions.iterableAsScalaIterable(futures))(fun.apply _)
|
2011-03-22 22:12:16 +01:00
|
|
|
|
2011-06-18 23:23:47 -06:00
|
|
|
def fold[T <: AnyRef, R <: AnyRef](zero: R, timeout: Long, futures: java.lang.Iterable[Future[T]], fun: akka.japi.Function2[R, T, R]): Future[R] = fold(zero, timeout: Timeout, futures, fun)
|
|
|
|
|
|
|
|
|
|
def fold[T <: AnyRef, R <: AnyRef](zero: R, futures: java.lang.Iterable[Future[T]], fun: akka.japi.Function2[R, T, R]): Future[R] = fold(zero, Timeout.default, futures, fun)
|
|
|
|
|
|
2011-01-24 16:37:08 +01:00
|
|
|
/**
|
|
|
|
|
* Initiates a fold over the supplied futures where the fold-zero is the result value of the Future that's completed first
|
2011-03-30 21:19:26 +02:00
|
|
|
* Example:
|
|
|
|
|
* <pre>
|
|
|
|
|
* val result = Futures.reduce(futures)(_ + _).await.result
|
|
|
|
|
* </pre>
|
2011-01-24 16:37:08 +01:00
|
|
|
*/
|
2011-06-18 23:23:47 -06:00
|
|
|
def reduce[T, R >: T](futures: Iterable[Future[T]], timeout: Timeout = Timeout.default)(op: (R, T) ⇒ T): Future[R] = {
|
2011-01-24 17:12:56 +01:00
|
|
|
if (futures.isEmpty)
|
2011-05-23 11:31:01 +02:00
|
|
|
new KeptPromise[R](Left(new UnsupportedOperationException("empty reduce left")))
|
2011-02-08 16:14:40 +01:00
|
|
|
else {
|
2011-05-23 11:31:01 +02:00
|
|
|
val result = new DefaultPromise[R](timeout)
|
2011-02-08 16:14:40 +01:00
|
|
|
val seedFound = new AtomicBoolean(false)
|
2011-05-18 17:25:30 +02:00
|
|
|
val seedFold: Future[T] ⇒ Unit = f ⇒ {
|
2011-03-18 01:21:26 +01:00
|
|
|
if (seedFound.compareAndSet(false, true)) { //Only the first completed should trigger the fold
|
|
|
|
|
f.value.get match {
|
2011-07-19 19:28:46 +02:00
|
|
|
case Right(value) ⇒ result.completeWith(fold(value, timeout)(futures.filterNot(_ eq f))(op))
|
|
|
|
|
case Left(exception) ⇒ result.completeWithException(exception)
|
2011-03-18 01:21:26 +01:00
|
|
|
}
|
2011-02-08 16:14:40 +01:00
|
|
|
}
|
2011-01-24 16:37:08 +01:00
|
|
|
}
|
2011-05-18 17:25:30 +02:00
|
|
|
for (f ← futures) f onComplete seedFold //Attach the listener to the Futures
|
2011-02-08 16:14:40 +01:00
|
|
|
result
|
2011-01-24 16:37:08 +01: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
|
|
|
|
|
*/
|
2011-06-18 23:23:47 -06:00
|
|
|
def reduce[T <: AnyRef, R >: T](futures: java.lang.Iterable[Future[T]], timeout: Timeout, fun: akka.japi.Function2[R, T, T]): Future[R] =
|
2011-04-07 13:03:23 +02:00
|
|
|
reduce(scala.collection.JavaConversions.iterableAsScalaIterable(futures), timeout)(fun.apply _)
|
2011-03-22 22:12:16 +01:00
|
|
|
|
2011-06-18 23:23:47 -06:00
|
|
|
def reduce[T <: AnyRef, R >: T](futures: java.lang.Iterable[Future[T]], timeout: Long, fun: akka.japi.Function2[R, T, T]): Future[R] = reduce(futures, timeout: Timeout, fun)
|
|
|
|
|
|
2011-04-15 13:09:53 -06:00
|
|
|
/**
|
|
|
|
|
* Java API.
|
2011-05-11 13:29:29 +02:00
|
|
|
* Simple version of Futures.traverse. Transforms a java.lang.Iterable[Future[A]] into a Future[java.lang.Iterable[A]].
|
2011-04-15 13:09:53 -06:00
|
|
|
* Useful for reducing many Futures into a single Future.
|
|
|
|
|
*/
|
2011-06-18 23:23:47 -06:00
|
|
|
def sequence[A](in: JIterable[Future[A]], timeout: Timeout): Future[JIterable[A]] =
|
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-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-05-11 13:29:29 +02:00
|
|
|
* Simple version of Futures.traverse. Transforms a java.lang.Iterable[Future[A]] into a Future[java.lang.Iterable[A]].
|
2011-04-03 10:40:06 -06:00
|
|
|
* Useful for reducing many Futures into a single Future.
|
2011-03-30 21:19:26 +02:00
|
|
|
*/
|
2011-06-18 23:23:47 -06:00
|
|
|
def sequence[A](in: JIterable[Future[A]]): Future[JIterable[A]] = sequence(in, Timeout.default)
|
2011-02-25 07:30:31 -07:00
|
|
|
|
2011-03-30 21:19:26 +02:00
|
|
|
/**
|
2011-04-15 13:09:53 -06:00
|
|
|
* Java API.
|
2011-05-11 13:29:29 +02:00
|
|
|
* Transforms a java.lang.Iterable[A] into a Future[java.lang.Iterable[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-06-18 23:23:47 -06:00
|
|
|
def traverse[A, B](in: JIterable[A], timeout: Timeout, fn: JFunc[A, Future[B]]): Future[JIterable[B]] =
|
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-05-18 17:25:30 +02:00
|
|
|
for (r ← fr; b ← fb) yield {
|
2011-04-15 13:09:53 -06:00
|
|
|
r add b
|
|
|
|
|
r
|
|
|
|
|
}
|
|
|
|
|
}
|
|
|
|
|
|
|
|
|
|
/**
|
|
|
|
|
* Java API.
|
2011-05-11 13:29:29 +02:00
|
|
|
* Transforms a java.lang.Iterable[A] into a Future[java.lang.Iterable[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.
|
2011-05-16 12:32:00 +02:00
|
|
|
*
|
2011-05-18 17:25:30 +02:00
|
|
|
* def traverse[A, B, M[_] <: Traversable[_]](in: M[A], timeout: Long = Actor.TIMEOUT)(fn: A => Future[B])(implicit cbf: CanBuildFrom[M[A], B, M[B]]): Future[M[B]] =
|
2011-05-23 11:31:01 +02:00
|
|
|
* in.foldLeft(new DefaultPromise[Builder[B, M[B]]](timeout).completeWithResult(cbf(in)): Future[Builder[B, M[B]]]) { (fr, a) =>
|
2011-05-18 17:25:30 +02:00
|
|
|
* val fb = fn(a.asInstanceOf[A])
|
|
|
|
|
* for (r <- fr; b <-fb) yield (r += b)
|
|
|
|
|
* }.map(_.result)
|
|
|
|
|
*/
|
2011-06-18 23:23:47 -06:00
|
|
|
def traverse[A, B](in: JIterable[A], fn: JFunc[A, Future[B]]): Future[JIterable[B]] = traverse(in, Timeout.default, fn)
|
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-07-26 22:23:16 -06:00
|
|
|
def apply[T](body: ⇒ T)(implicit dispatcher: MessageDispatcher, timeout: Timeout = implicitly): Future[T] = {
|
|
|
|
|
val promise = new DefaultPromise[T](timeout)
|
|
|
|
|
dispatcher dispatchTask { () ⇒
|
|
|
|
|
promise complete {
|
|
|
|
|
try {
|
|
|
|
|
Right(body)
|
|
|
|
|
} catch {
|
|
|
|
|
case e ⇒ Left(e)
|
|
|
|
|
}
|
|
|
|
|
}
|
|
|
|
|
}
|
|
|
|
|
promise
|
|
|
|
|
}
|
2011-04-15 13:09:53 -06:00
|
|
|
|
2011-06-25 14:20:54 -06:00
|
|
|
def apply[T](body: ⇒ T, timeout: Timeout)(implicit dispatcher: MessageDispatcher): Future[T] =
|
|
|
|
|
apply(body)(dispatcher, timeout)
|
|
|
|
|
|
2011-08-02 13:01:10 -06:00
|
|
|
def apply[T](body: ⇒ T, timeout: Duration)(implicit dispatcher: MessageDispatcher): Future[T] =
|
|
|
|
|
apply(body)(dispatcher, timeout)
|
|
|
|
|
|
2011-06-25 14:20:54 -06:00
|
|
|
def apply[T](body: ⇒ T, timeout: Long)(implicit dispatcher: MessageDispatcher): Future[T] =
|
|
|
|
|
apply(body)(dispatcher, timeout)
|
|
|
|
|
|
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-06-25 14:20:54 -06:00
|
|
|
def sequence[A, M[_] <: Traversable[_]](in: M[Future[A]])(implicit cbf: CanBuildFrom[M[Future[A]], A, M[A]], timeout: Timeout): Future[M[A]] =
|
|
|
|
|
in.foldLeft(new KeptPromise(Right(cbf(in))): Future[Builder[A, M[A]]])((fr, fa) ⇒ for (r ← fr; a ← fa.asInstanceOf[Future[A]]) yield (r += a)).map(_.result)
|
|
|
|
|
|
|
|
|
|
def sequence[A, M[_] <: Traversable[_]](timeout: Timeout)(in: M[Future[A]])(implicit cbf: CanBuildFrom[M[Future[A]], A, M[A]]): Future[M[A]] =
|
|
|
|
|
sequence(in)(cbf, timeout)
|
2011-04-15 13:09:53 -06:00
|
|
|
|
|
|
|
|
/**
|
|
|
|
|
* Transforms a Traversable[A] into a Future[Traversable[B]] using the provided Function A => Future[B].
|
|
|
|
|
* This is useful for performing a parallel map. For example, to apply a function to all items of a list
|
|
|
|
|
* in parallel:
|
|
|
|
|
* <pre>
|
|
|
|
|
* val myFutureList = Futures.traverse(myList)(x => Future(myFunc(x)))
|
|
|
|
|
* </pre>
|
|
|
|
|
*/
|
2011-06-25 14:20:54 -06:00
|
|
|
def traverse[A, B, M[_] <: Traversable[_]](in: M[A])(fn: A ⇒ Future[B])(implicit cbf: CanBuildFrom[M[A], B, M[B]], timeout: Timeout): Future[M[B]] =
|
|
|
|
|
in.foldLeft(new KeptPromise(Right(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-04-21 15:12:47 -06:00
|
|
|
|
2011-06-25 14:20:54 -06:00
|
|
|
def traverse[A, B, M[_] <: Traversable[_]](in: M[A], timeout: Timeout)(fn: A ⇒ Future[B])(implicit cbf: CanBuildFrom[M[A], B, M[B]]): Future[M[B]] =
|
|
|
|
|
traverse(in)(fn)(cbf, timeout)
|
|
|
|
|
|
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-07-26 22:23:16 -06:00
|
|
|
def flow[A](body: ⇒ A @cps[Future[Any]])(implicit dispatcher: MessageDispatcher, timeout: Timeout): Future[A] = {
|
2011-05-03 18:48:42 -06:00
|
|
|
val future = Promise[A](timeout)
|
2011-07-26 22:23:16 -06:00
|
|
|
//dispatcher dispatchTask { () ⇒
|
2011-08-02 13:38:02 -06:00
|
|
|
(reify(body) foreachFull (future completeWithResult, future completeWithException): Future[Any]) onException {
|
2011-07-26 22:23:16 -06:00
|
|
|
case e: Exception ⇒ future completeWithException e
|
|
|
|
|
}
|
|
|
|
|
//}
|
2011-05-02 16:56:42 -06:00
|
|
|
future
|
2011-05-02 14:44:40 -06:00
|
|
|
}
|
2011-02-28 11:47:39 -07:00
|
|
|
}
|
|
|
|
|
|
2011-03-11 10:46:36 -07:00
|
|
|
sealed trait Future[+T] {
|
2011-03-22 22:12:16 +01:00
|
|
|
|
2011-08-02 10:19:49 -06:00
|
|
|
implicit def dispatcher: MessageDispatcher
|
|
|
|
|
|
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.
|
|
|
|
|
*
|
|
|
|
|
* If this Future is untyped (a Future[Nothing]), a type parameter must be explicitly provided or
|
2011-06-13 13:43:21 +02:00
|
|
|
* execution will fail. The normal result of getting a Future from an ActorRef using ? will return
|
2011-04-23 12:57:28 -06:00
|
|
|
* an untyped Future.
|
2011-03-22 22:12:16 +01:00
|
|
|
*/
|
2011-08-02 10:19:49 -06:00
|
|
|
def apply[A >: T]()(implicit timeout: Timeout): A @cps[Future[Any]] = shift(this flatMap (_: A ⇒ Future[Any]))
|
2011-03-22 22:12:16 +01:00
|
|
|
|
|
|
|
|
/**
|
2011-04-26 10:49:52 +02:00
|
|
|
* Blocks awaiting completion of this Future, then returns the resulting value,
|
|
|
|
|
* or throws the completed exception
|
|
|
|
|
*
|
|
|
|
|
* Scala & Java API
|
|
|
|
|
*
|
|
|
|
|
* throws FutureTimeoutException if this Future times out when waiting for completion
|
2011-03-22 22:12:16 +01:00
|
|
|
*/
|
2011-04-21 15:12:47 -06:00
|
|
|
def get: T = this.await.resultOrException.get
|
2011-03-22 22:12:16 +01:00
|
|
|
|
2011-02-23 19:57:42 -07:00
|
|
|
/**
|
|
|
|
|
* Blocks the current thread until the Future has been completed or the
|
|
|
|
|
* timeout has expired. In the case of the timeout expiring a
|
|
|
|
|
* FutureTimeoutException will be thrown.
|
|
|
|
|
*/
|
2011-05-18 17:25:30 +02:00
|
|
|
def await: Future[T]
|
2010-10-31 19:27:55 +01:00
|
|
|
|
2011-04-27 15:34:42 +02:00
|
|
|
/**
|
|
|
|
|
* Blocks the current thread until the Future has been completed or the
|
|
|
|
|
* timeout has expired. The timeout will be the least value of 'atMost' and the timeout
|
|
|
|
|
* supplied at the constructuion of this Future.
|
|
|
|
|
* In the case of the timeout expiring a FutureTimeoutException will be thrown.
|
|
|
|
|
*/
|
2011-05-18 17:25:30 +02:00
|
|
|
def await(atMost: Duration): Future[T]
|
2011-04-27 15:34:42 +02:00
|
|
|
|
2011-02-23 19:57:42 -07:00
|
|
|
/**
|
2011-06-13 22:36:46 +02:00
|
|
|
* Await completion of this Future (as `await`) and return its value if it
|
|
|
|
|
* conforms to A's erased type.
|
2011-06-14 14:26:13 +02:00
|
|
|
*
|
|
|
|
|
* def as[A](implicit m: Manifest[A]): Option[A] =
|
|
|
|
|
* try {
|
|
|
|
|
* await
|
|
|
|
|
* value match {
|
|
|
|
|
* case None ⇒ None
|
|
|
|
|
* case Some(_: Left[_, _]) ⇒ None
|
|
|
|
|
* case Some(Right(v)) ⇒ Some(BoxedType(m.erasure).cast(v).asInstanceOf[A])
|
|
|
|
|
* }
|
|
|
|
|
* } catch {
|
|
|
|
|
* case _: Exception ⇒ None
|
|
|
|
|
* }
|
2011-02-23 19:57:42 -07:00
|
|
|
*/
|
2010-10-31 19:27:55 +01:00
|
|
|
|
2011-02-23 19:57:42 -07:00
|
|
|
/**
|
|
|
|
|
* Tests whether this Future has been completed.
|
|
|
|
|
*/
|
2011-02-13 19:59:54 -07:00
|
|
|
final def isCompleted: Boolean = value.isDefined
|
2010-10-31 19:27:55 +01:00
|
|
|
|
2011-02-23 19:57:42 -07:00
|
|
|
/**
|
|
|
|
|
* Tests whether this Future's timeout has expired.
|
|
|
|
|
*
|
|
|
|
|
* Note that an expired Future may still contain a value, or it may be
|
|
|
|
|
* completed with a value.
|
|
|
|
|
*/
|
2009-05-25 14:48:43 +02:00
|
|
|
def isExpired: Boolean
|
2010-10-31 19:27:55 +01:00
|
|
|
|
2011-06-18 23:23:47 -06:00
|
|
|
def timeout: Timeout
|
|
|
|
|
|
2011-02-23 19:57:42 -07:00
|
|
|
/**
|
|
|
|
|
* This Future's timeout in nanoseconds.
|
|
|
|
|
*/
|
2011-06-18 23:23:47 -06:00
|
|
|
def timeoutInNanos = if (timeout.duration.isFinite) timeout.duration.toNanos else Long.MaxValue
|
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
|
|
|
/**
|
|
|
|
|
* Returns the successful result of this Future if it exists.
|
|
|
|
|
*/
|
2011-02-13 19:59:54 -07:00
|
|
|
final def result: Option[T] = {
|
|
|
|
|
val v = value
|
|
|
|
|
if (v.isDefined) v.get.right.toOption
|
|
|
|
|
else None
|
|
|
|
|
}
|
2010-10-31 19:27:55 +01:00
|
|
|
|
2011-02-23 19:57:42 -07:00
|
|
|
/**
|
|
|
|
|
* Returns the contained exception of this Future if it exists.
|
|
|
|
|
*/
|
2011-02-13 19:59:54 -07:00
|
|
|
final def exception: Option[Throwable] = {
|
|
|
|
|
val v = value
|
|
|
|
|
if (v.isDefined) v.get.left.toOption
|
|
|
|
|
else None
|
|
|
|
|
}
|
2010-10-31 19:27:55 +01:00
|
|
|
|
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-04-23 08:11:31 +02:00
|
|
|
* immediately.
|
2011-02-23 19:57:42 -07:00
|
|
|
*/
|
2011-06-13 22:36:46 +02:00
|
|
|
def onComplete(func: Future[T] ⇒ Unit): 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-02-23 19:57:42 -07:00
|
|
|
* PartialFunction to the result.
|
2011-03-30 21:19:26 +02:00
|
|
|
* <pre>
|
2011-06-02 13:33:49 -07:00
|
|
|
* val result = future onResult {
|
2011-03-30 21:19:26 +02:00
|
|
|
* case Foo => "foo"
|
|
|
|
|
* case Bar => "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-08-03 14:26:01 +02:00
|
|
|
final def onResult(pf: PartialFunction[T, Unit]): this.type = onComplete { f ⇒
|
2011-02-13 21:11:37 -07:00
|
|
|
val optr = f.result
|
|
|
|
|
if (optr.isDefined) {
|
|
|
|
|
val r = optr.get
|
2011-06-02 13:33:49 -07:00
|
|
|
if (pf isDefinedAt r) pf(r)
|
|
|
|
|
}
|
|
|
|
|
}
|
|
|
|
|
|
|
|
|
|
/**
|
|
|
|
|
* When the future is completed with an exception, apply the provided
|
|
|
|
|
* PartialFunction to the exception.
|
|
|
|
|
* <pre>
|
|
|
|
|
* val result = future onException {
|
|
|
|
|
* case Foo => "foo"
|
|
|
|
|
* case Bar => "bar"
|
|
|
|
|
* }
|
|
|
|
|
* </pre>
|
|
|
|
|
*/
|
|
|
|
|
final def onException(pf: PartialFunction[Throwable, Unit]): Future[T] = onComplete { f ⇒
|
|
|
|
|
val opte = f.exception
|
|
|
|
|
if (opte.isDefined) {
|
|
|
|
|
val e = opte.get
|
|
|
|
|
if (pf isDefinedAt e) pf(e)
|
2011-02-13 21:11:37 -07:00
|
|
|
}
|
|
|
|
|
}
|
|
|
|
|
|
2011-06-17 06:51:40 -06:00
|
|
|
def onTimeout(func: Future[T] ⇒ Unit): this.type
|
|
|
|
|
|
2011-06-20 16:22:46 -06:00
|
|
|
def orElse[A >: T](fallback: ⇒ A): Future[A]
|
|
|
|
|
|
2011-03-11 10:46:36 -07:00
|
|
|
/**
|
|
|
|
|
* Creates a new Future by applying a PartialFunction to the successful
|
|
|
|
|
* result of this Future if a match is found, or else return a MatchError.
|
|
|
|
|
* 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 <- actor ? Req("Hello") collect { case Res(x: Int) => x }
|
|
|
|
|
* b <- actor ? Req(a) collect { case Res(x: String) => x }
|
|
|
|
|
* c <- actor ? Req(7) collect { case Res(x: String) => x }
|
2011-03-30 21:19:26 +02:00
|
|
|
* } yield b + "-" + c
|
|
|
|
|
* </pre>
|
2011-03-11 10:46:36 -07:00
|
|
|
*/
|
2011-08-02 10:19:49 -06:00
|
|
|
final def collect[A](pf: PartialFunction[Any, A])(implicit timeout: Timeout): Future[A] = {
|
2011-07-27 22:20:02 -06:00
|
|
|
val future = new DefaultPromise[A](timeout)
|
|
|
|
|
onComplete { self ⇒
|
|
|
|
|
future complete {
|
|
|
|
|
self.value.get match {
|
|
|
|
|
case Right(r) ⇒
|
|
|
|
|
try {
|
|
|
|
|
if (pf isDefinedAt r) Right(pf(r))
|
|
|
|
|
else Left(new MatchError(r))
|
|
|
|
|
} catch {
|
|
|
|
|
case e: Exception ⇒
|
|
|
|
|
EventHandler.error(e, this, e.getMessage)
|
|
|
|
|
Left(e)
|
|
|
|
|
}
|
|
|
|
|
case v ⇒ v.asInstanceOf[Either[Throwable, A]]
|
2011-06-18 20:13:13 -06:00
|
|
|
}
|
|
|
|
|
}
|
2011-07-27 22:20:02 -06:00
|
|
|
}
|
|
|
|
|
future
|
2011-06-18 20:13:13 -06:00
|
|
|
}
|
2011-03-11 10:46:36 -07: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-06-02 13:33:49 -07: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-08-02 10:19:49 -06:00
|
|
|
final def recover[A >: T](pf: PartialFunction[Throwable, A])(implicit timeout: Timeout): Future[A] = {
|
2011-07-27 22:20:02 -06:00
|
|
|
val future = new DefaultPromise[A](timeout)
|
|
|
|
|
onComplete { self ⇒
|
|
|
|
|
future complete {
|
|
|
|
|
self.value.get match {
|
|
|
|
|
case Left(e) ⇒
|
|
|
|
|
try {
|
|
|
|
|
if (pf isDefinedAt e) Right(pf(e))
|
|
|
|
|
else Left(e)
|
|
|
|
|
} catch {
|
|
|
|
|
case x: Exception ⇒
|
|
|
|
|
Left(x)
|
|
|
|
|
}
|
|
|
|
|
case v ⇒ v
|
2011-06-18 20:13:13 -06:00
|
|
|
}
|
|
|
|
|
}
|
2011-07-27 22:20:02 -06:00
|
|
|
}
|
|
|
|
|
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-08-02 10:19:49 -06:00
|
|
|
final def map[A](f: T ⇒ A)(implicit timeout: Timeout): Future[A] = {
|
2011-07-27 22:20:02 -06:00
|
|
|
val future = new DefaultPromise[A](timeout)
|
|
|
|
|
onComplete { self ⇒
|
|
|
|
|
future complete {
|
|
|
|
|
self.value.get match {
|
|
|
|
|
case Right(r) ⇒
|
|
|
|
|
try {
|
|
|
|
|
Right(f(r))
|
|
|
|
|
} catch {
|
|
|
|
|
case e: Exception ⇒
|
|
|
|
|
EventHandler.error(e, this, e.getMessage)
|
|
|
|
|
Left(e)
|
|
|
|
|
}
|
|
|
|
|
case v ⇒ v.asInstanceOf[Either[Throwable, A]]
|
2011-06-18 20:13:13 -06:00
|
|
|
}
|
|
|
|
|
}
|
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.
|
|
|
|
|
*/
|
2011-08-02 10:19:49 -06:00
|
|
|
final def mapTo[A](implicit m: Manifest[A], timeout: Timeout = this.timeout): Future[A] = {
|
2011-07-27 22:20:02 -06:00
|
|
|
val fa = new DefaultPromise[A](timeout)
|
|
|
|
|
onComplete { ft ⇒
|
|
|
|
|
fa complete (ft.value.get match {
|
|
|
|
|
case l: Left[_, _] ⇒ l.asInstanceOf[Either[Throwable, A]]
|
|
|
|
|
case Right(t) ⇒
|
|
|
|
|
try {
|
|
|
|
|
Right(BoxedType(m.erasure).cast(t).asInstanceOf[A])
|
|
|
|
|
} catch {
|
|
|
|
|
case e: ClassCastException ⇒ Left(e)
|
|
|
|
|
}
|
2011-06-18 20:13:13 -06:00
|
|
|
})
|
2011-07-27 22:20:02 -06:00
|
|
|
}
|
|
|
|
|
fa
|
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-08-02 10:19:49 -06:00
|
|
|
final def flatMap[A](f: T ⇒ Future[A])(implicit timeout: Timeout): Future[A] = {
|
2011-07-27 22:20:02 -06:00
|
|
|
val future = new DefaultPromise[A](timeout)
|
|
|
|
|
onComplete {
|
|
|
|
|
_.value.get match {
|
|
|
|
|
case Right(r) ⇒
|
|
|
|
|
try {
|
|
|
|
|
future completeWith f(r)
|
|
|
|
|
} catch {
|
|
|
|
|
case e: Exception ⇒
|
|
|
|
|
EventHandler.error(e, this, e.getMessage)
|
|
|
|
|
future complete Left(e)
|
|
|
|
|
}
|
|
|
|
|
case v ⇒ future complete v.asInstanceOf[Either[Throwable, A]]
|
2011-06-18 20:13:13 -06:00
|
|
|
}
|
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-05-26 19:33:03 +02:00
|
|
|
final def foreach(f: T ⇒ Unit): Unit = onComplete {
|
|
|
|
|
_.result match {
|
|
|
|
|
case Some(v) ⇒ f(v)
|
|
|
|
|
case None ⇒
|
|
|
|
|
}
|
|
|
|
|
}
|
|
|
|
|
|
2011-08-02 10:19:49 -06:00
|
|
|
final def withFilter(p: T ⇒ Boolean)(implicit timeout: Timeout) = new FutureWithFilter[T](this, p)
|
2011-05-26 19:33:03 +02:00
|
|
|
|
2011-06-25 14:20:54 -06:00
|
|
|
final class FutureWithFilter[+A](self: Future[A], p: A ⇒ Boolean)(implicit timeout: Timeout) {
|
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-08-02 10:19:49 -06:00
|
|
|
final def filter(p: T ⇒ Boolean)(implicit timeout: Timeout): Future[T] = {
|
2011-07-27 22:20:02 -06:00
|
|
|
val future = new DefaultPromise[T](timeout)
|
|
|
|
|
onComplete { self ⇒
|
|
|
|
|
future complete {
|
|
|
|
|
self.value.get match {
|
|
|
|
|
case Right(r) ⇒
|
|
|
|
|
try {
|
|
|
|
|
if (p(r))
|
|
|
|
|
Right(r)
|
|
|
|
|
else
|
|
|
|
|
Left(new MatchError(r))
|
|
|
|
|
} catch {
|
|
|
|
|
case e: Exception ⇒
|
|
|
|
|
EventHandler.error(e, this, e.getMessage)
|
|
|
|
|
Left(e)
|
|
|
|
|
}
|
|
|
|
|
case v ⇒ v
|
2011-06-18 20:13:13 -06:00
|
|
|
}
|
|
|
|
|
}
|
2011-07-27 22:20:02 -06:00
|
|
|
}
|
|
|
|
|
future
|
2011-06-18 20:13:13 -06:00
|
|
|
}
|
2011-02-21 18:28:17 -07:00
|
|
|
|
2010-11-12 14:04:06 +01:00
|
|
|
/**
|
2011-06-16 08:08:31 -07:00
|
|
|
* Returns the current result, throws the exception if one has been raised, else returns None
|
2010-11-12 14:04:06 +01:00
|
|
|
*/
|
2011-02-13 19:59:54 -07:00
|
|
|
final def resultOrException: Option[T] = {
|
|
|
|
|
val v = value
|
|
|
|
|
if (v.isDefined) {
|
|
|
|
|
val r = v.get
|
|
|
|
|
if (r.isLeft) throw r.left.get
|
|
|
|
|
else r.right.toOption
|
|
|
|
|
} else None
|
|
|
|
|
}
|
2010-11-12 14:04:06 +01:00
|
|
|
|
2010-11-12 12:11:53 +01:00
|
|
|
/* Java API */
|
2011-06-13 22:31:06 -06:00
|
|
|
final def onComplete[A >: T](proc: Procedure[Future[A]]): this.type = onComplete(proc(_))
|
2011-03-11 10:46:36 -07:00
|
|
|
|
2011-05-18 17:25:30 +02:00
|
|
|
final def map[A >: T, B](f: JFunc[A, B]): Future[B] = map(f(_))
|
2011-03-11 10:46:36 -07:00
|
|
|
|
2011-05-18 17:25:30 +02:00
|
|
|
final def flatMap[A >: T, B](f: JFunc[A, Future[B]]): Future[B] = flatMap(f(_))
|
2011-03-11 10:46:36 -07:00
|
|
|
|
|
|
|
|
final def foreach[A >: T](proc: Procedure[A]): Unit = foreach(proc(_))
|
|
|
|
|
|
2011-05-18 17:25:30 +02:00
|
|
|
final def filter(p: JFunc[Any, Boolean]): Future[Any] = filter(p(_))
|
2010-11-12 12:11:53 +01:00
|
|
|
|
2009-05-25 14:48:43 +02:00
|
|
|
}
|
|
|
|
|
|
2011-05-03 18:48:42 -06:00
|
|
|
object Promise {
|
|
|
|
|
|
2011-06-17 09:54:49 +02:00
|
|
|
/**
|
|
|
|
|
* Creates a non-completed, new, Promise with the supplied timeout in milliseconds
|
|
|
|
|
*/
|
2011-08-02 10:19:49 -06:00
|
|
|
def apply[A](timeout: Timeout)(implicit dispatcher: MessageDispatcher): Promise[A] = new DefaultPromise[A](timeout)
|
2011-05-03 18:48:42 -06:00
|
|
|
|
2011-06-17 09:54:49 +02:00
|
|
|
/**
|
|
|
|
|
* Creates a non-completed, new, Promise with the default timeout (akka.actor.timeout in conf)
|
|
|
|
|
*/
|
2011-08-02 10:19:49 -06:00
|
|
|
def apply[A]()(implicit dispatcher: MessageDispatcher): Promise[A] = apply(Timeout.default)
|
2010-11-12 12:11:53 +01:00
|
|
|
|
2011-06-04 12:42:06 -07:00
|
|
|
/**
|
|
|
|
|
* Construct a completable channel
|
|
|
|
|
*/
|
2011-08-02 10:19:49 -06:00
|
|
|
def channel(timeout: Long = Actor.TIMEOUT)(implicit dispatcher: MessageDispatcher): ActorPromise = new ActorPromise(timeout)
|
2011-06-04 12:42:06 -07:00
|
|
|
|
2011-05-23 11:54:41 +02:00
|
|
|
private[akka] val callbacksPendingExecution = new ThreadLocal[Option[Stack[() ⇒ Unit]]]() {
|
|
|
|
|
override def initialValue = None
|
|
|
|
|
}
|
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-03-22 22:12:16 +01:00
|
|
|
/**
|
2011-03-30 21:19:26 +02:00
|
|
|
* Completes this Future with the specified result, if not already completed.
|
|
|
|
|
* @return this
|
2011-03-22 22:12:16 +01:00
|
|
|
*/
|
2011-06-13 22:36:46 +02:00
|
|
|
def complete(value: Either[Throwable, T]): this.type
|
2011-03-22 22:12:16 +01:00
|
|
|
|
|
|
|
|
/**
|
2011-03-30 21:19:26 +02:00
|
|
|
* Completes this Future with the specified result, if not already completed.
|
|
|
|
|
* @return this
|
2011-03-22 22:12:16 +01:00
|
|
|
*/
|
2011-06-13 22:36:46 +02:00
|
|
|
final def completeWithResult(result: T): this.type = complete(Right(result))
|
2011-03-22 22:12:16 +01:00
|
|
|
|
|
|
|
|
/**
|
2011-03-30 21:19:26 +02:00
|
|
|
* Completes this Future with the specified exception, if not already completed.
|
|
|
|
|
* @return this
|
2011-03-22 22:12:16 +01:00
|
|
|
*/
|
2011-06-13 22:36:46 +02:00
|
|
|
final def completeWithException(exception: Throwable): this.type = complete(Left(exception))
|
2011-03-22 22:12:16 +01:00
|
|
|
|
|
|
|
|
/**
|
|
|
|
|
* Completes this Future with the specified other Future, when that Future is completed,
|
2011-03-30 21:19:26 +02:00
|
|
|
* unless this Future has already been completed.
|
|
|
|
|
* @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 = {
|
2011-05-18 17:25:30 +02:00
|
|
|
other onComplete { f ⇒ complete(f.value.get) }
|
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-07-26 22:23:16 -06:00
|
|
|
val fr = new DefaultPromise[Any](this.timeout)
|
2011-05-18 17:25:30 +02:00
|
|
|
this completeWith other onComplete { f ⇒
|
2011-04-23 09:14:20 -06:00
|
|
|
try {
|
|
|
|
|
fr completeWith cont(f)
|
|
|
|
|
} catch {
|
2011-05-18 17:25:30 +02:00
|
|
|
case e: Exception ⇒
|
2011-04-23 09:14:20 -06:00
|
|
|
EventHandler.error(e, this, e.getMessage)
|
|
|
|
|
fr completeWithException e
|
|
|
|
|
}
|
|
|
|
|
}
|
|
|
|
|
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-07-26 22:23:16 -06:00
|
|
|
val fr = new DefaultPromise[Any](this.timeout)
|
2011-05-19 14:43:58 -06:00
|
|
|
stream.dequeue(this).onComplete { f ⇒
|
|
|
|
|
try {
|
|
|
|
|
fr completeWith cont(f)
|
|
|
|
|
} catch {
|
|
|
|
|
case e: Exception ⇒
|
|
|
|
|
EventHandler.error(e, this, e.getMessage)
|
|
|
|
|
fr completeWithException e
|
|
|
|
|
}
|
|
|
|
|
}
|
|
|
|
|
fr
|
|
|
|
|
}
|
|
|
|
|
|
2009-05-25 14:48:43 +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-08-02 10:19:49 -06:00
|
|
|
class DefaultPromise[T](val timeout: Timeout)(implicit val dispatcher: MessageDispatcher) extends Promise[T] {
|
2011-06-17 06:51:40 -06:00
|
|
|
self ⇒
|
2010-10-31 19:27:55 +01:00
|
|
|
|
2011-08-02 10:19:49 -06:00
|
|
|
def this()(implicit dispatcher: MessageDispatcher) = this(Timeout.default)
|
2011-06-18 23:23:47 -06:00
|
|
|
|
2011-08-02 10:19:49 -06:00
|
|
|
def this(timeout: Long)(implicit dispatcher: MessageDispatcher) = this(Timeout(timeout))
|
2009-06-05 22:08:53 +02:00
|
|
|
|
2011-08-02 10:19:49 -06:00
|
|
|
def this(timeout: Long, timeunit: TimeUnit)(implicit dispatcher: MessageDispatcher) = this(Timeout(timeout, timeunit))
|
2011-02-12 12:26:56 -07:00
|
|
|
|
2009-05-25 14:48:43 +02:00
|
|
|
private val _startTimeInNanos = currentTimeInNanos
|
|
|
|
|
private val _lock = new ReentrantLock
|
|
|
|
|
private val _signal = _lock.newCondition
|
2011-02-11 14:46:39 -07:00
|
|
|
private var _value: Option[Either[Throwable, T]] = None
|
2011-05-18 17:25:30 +02:00
|
|
|
private var _listeners: List[Future[T] ⇒ Unit] = Nil
|
2009-06-21 14:08:43 +02:00
|
|
|
|
2011-04-26 17:19:07 +02:00
|
|
|
/**
|
|
|
|
|
* Must be called inside _lock.lock<->_lock.unlock
|
2011-08-01 12:33:40 +02:00
|
|
|
* Returns true if completed within the timeout
|
2011-04-26 17:19:07 +02:00
|
|
|
*/
|
2011-02-12 12:26:56 -07:00
|
|
|
@tailrec
|
2011-04-27 15:34:42 +02:00
|
|
|
private def awaitUnsafe(waitTimeNanos: Long): Boolean = {
|
|
|
|
|
if (_value.isEmpty && waitTimeNanos > 0) {
|
2011-02-07 18:59:49 +01:00
|
|
|
val start = currentTimeInNanos
|
2011-04-27 15:34:42 +02:00
|
|
|
val remainingNanos = try {
|
|
|
|
|
_signal.awaitNanos(waitTimeNanos)
|
2011-02-07 18:59:49 +01:00
|
|
|
} catch {
|
2011-05-18 17:25:30 +02:00
|
|
|
case e: InterruptedException ⇒
|
2011-04-27 15:34:42 +02:00
|
|
|
waitTimeNanos - (currentTimeInNanos - start)
|
2011-02-28 10:17:44 -07:00
|
|
|
}
|
2011-04-27 15:34:42 +02:00
|
|
|
awaitUnsafe(remainingNanos)
|
2011-02-12 09:16:29 -07:00
|
|
|
} else {
|
|
|
|
|
_value.isDefined
|
2011-02-07 18:59:49 +01:00
|
|
|
}
|
2011-02-12 09:16:29 -07:00
|
|
|
}
|
|
|
|
|
|
2011-04-27 15:34:42 +02:00
|
|
|
def await(atMost: Duration) = {
|
2011-05-18 08:37:58 +02:00
|
|
|
_lock.lock()
|
2011-06-18 23:23:47 -06:00
|
|
|
try {
|
2011-08-01 12:33:40 +02:00
|
|
|
if (!atMost.isFinite && !timeout.duration.isFinite) { //If wait until infinity
|
2011-06-18 23:23:47 -06:00
|
|
|
while (_value.isEmpty) { _signal.await }
|
|
|
|
|
this
|
2011-08-01 12:33:40 +02:00
|
|
|
} else { //Limited wait
|
|
|
|
|
val time = if (!atMost.isFinite) timeLeft() //If atMost is infinity, use preset timeout
|
2011-08-01 15:23:57 +02:00
|
|
|
else if (!timeout.duration.isFinite) atMost.toNanos //If preset timeout is infinite, use atMost
|
|
|
|
|
else atMost.toNanos min timeLeft() //Otherwise use the smallest of them
|
2011-08-01 12:33:40 +02:00
|
|
|
if (awaitUnsafe(time)) this
|
|
|
|
|
else throw new FutureTimeoutException("Future timed out after [" + NANOS.toMillis(time) + "] ms")
|
2011-06-18 23:23:47 -06:00
|
|
|
}
|
2011-08-01 12:33:40 +02:00
|
|
|
} finally { _lock.unlock }
|
2009-05-25 14:48:43 +02:00
|
|
|
}
|
|
|
|
|
|
2011-08-01 12:33:40 +02:00
|
|
|
def await = await(timeout.duration)
|
|
|
|
|
|
2011-06-18 23:23:47 -06:00
|
|
|
def isExpired: Boolean = if (timeout.duration.isFinite) timeLeft() <= 0 else false
|
2009-05-25 14:48:43 +02:00
|
|
|
|
2011-02-28 10:17:44 -07:00
|
|
|
def value: Option[Either[Throwable, T]] = {
|
|
|
|
|
_lock.lock
|
|
|
|
|
try {
|
|
|
|
|
_value
|
|
|
|
|
} finally {
|
|
|
|
|
_lock.unlock
|
2009-05-25 14:48:43 +02:00
|
|
|
}
|
2011-02-28 10:17:44 -07:00
|
|
|
}
|
2010-11-12 12:11:53 +01:00
|
|
|
|
2011-06-13 22:36:46 +02:00
|
|
|
def complete(value: Either[Throwable, T]): this.type = {
|
2011-07-28 09:57:45 -06:00
|
|
|
processCallbacks {
|
|
|
|
|
_lock.lock
|
|
|
|
|
try {
|
|
|
|
|
if (_value.isEmpty) { //Only complete if we aren't expired
|
|
|
|
|
if (!isExpired) {
|
|
|
|
|
_value = Some(value)
|
|
|
|
|
val existingListeners = _listeners
|
|
|
|
|
_listeners = Nil
|
|
|
|
|
existingListeners
|
|
|
|
|
} else {
|
|
|
|
|
_listeners = Nil
|
|
|
|
|
Nil
|
|
|
|
|
}
|
|
|
|
|
} else Nil
|
|
|
|
|
} finally {
|
|
|
|
|
_signal.signalAll
|
|
|
|
|
_lock.unlock
|
|
|
|
|
}
|
2011-02-27 17:42:17 -07:00
|
|
|
}
|
2011-02-28 10:17:44 -07:00
|
|
|
|
2011-07-28 09:57:45 -06:00
|
|
|
this
|
|
|
|
|
}
|
|
|
|
|
|
2011-08-02 10:19:49 -06:00
|
|
|
private def processCallbacks(callbacks: List[Future[T] ⇒ Unit]): Unit = {
|
2011-07-28 09:57:45 -06:00
|
|
|
if (callbacks.nonEmpty) { // Steps to ensure we don't run into a stack-overflow situation
|
2011-05-18 17:25:30 +02:00
|
|
|
@tailrec
|
2011-07-28 09:57:45 -06:00
|
|
|
def runCallbacks(rest: List[Future[T] ⇒ Unit], callbackStack: Stack[() ⇒ Unit]) {
|
2011-04-29 13:22:39 +02:00
|
|
|
if (rest.nonEmpty) {
|
|
|
|
|
notifyCompleted(rest.head)
|
2011-07-28 09:57:45 -06:00
|
|
|
while (callbackStack.nonEmpty) { callbackStack.pop().apply() }
|
|
|
|
|
runCallbacks(rest.tail, callbackStack)
|
2011-04-29 13:22:39 +02:00
|
|
|
}
|
2011-04-28 21:14:18 -06:00
|
|
|
}
|
|
|
|
|
|
2011-05-23 11:54:41 +02:00
|
|
|
val pending = Promise.callbacksPendingExecution.get
|
2011-04-29 13:22:39 +02:00
|
|
|
if (pending.isDefined) { //Instead of nesting the calls to the callbacks (leading to stack overflow)
|
2011-05-18 17:25:30 +02:00
|
|
|
pending.get.push(() ⇒ { // Linearize/aggregate callbacks at top level and then execute
|
2011-04-29 13:22:39 +02:00
|
|
|
val doNotify = notifyCompleted _ //Hoist closure to avoid garbage
|
2011-07-28 09:57:45 -06:00
|
|
|
callbacks foreach doNotify
|
2011-04-29 13:22:39 +02:00
|
|
|
})
|
|
|
|
|
} else {
|
2011-07-26 22:23:16 -06:00
|
|
|
dispatcher dispatchTask { () ⇒
|
|
|
|
|
try {
|
2011-07-28 09:57:45 -06:00
|
|
|
val callbackStack = Stack[() ⇒ Unit]() // Allocate new aggregator for pending callbacks
|
|
|
|
|
Promise.callbacksPendingExecution.set(Some(callbackStack)) // Specify the callback aggregator
|
|
|
|
|
runCallbacks(callbacks, callbackStack) // Execute callbacks, if they trigger new callbacks, they are aggregated
|
2011-07-26 22:23:16 -06:00
|
|
|
} finally { Promise.callbacksPendingExecution.set(None) } // Ensure cleanup
|
|
|
|
|
}
|
2011-04-28 20:53:45 -06:00
|
|
|
}
|
|
|
|
|
}
|
2010-11-12 12:11:53 +01:00
|
|
|
}
|
|
|
|
|
|
2011-06-13 22:36:46 +02:00
|
|
|
def onComplete(func: Future[T] ⇒ Unit): this.type = {
|
2011-02-28 10:17:44 -07:00
|
|
|
_lock.lock
|
2011-03-18 17:37:25 +01:00
|
|
|
val notifyNow = try {
|
|
|
|
|
if (_value.isEmpty) {
|
2011-05-18 17:25:30 +02:00
|
|
|
if (!isExpired) { //Only add the listener if the future isn't expired
|
2011-04-27 16:52:19 +02:00
|
|
|
_listeners ::= func
|
|
|
|
|
false
|
|
|
|
|
} else false //Will never run the callback since the future is expired
|
2011-03-18 17:37:25 +01:00
|
|
|
} else true
|
2011-02-28 10:17:44 -07:00
|
|
|
} finally {
|
|
|
|
|
_lock.unlock
|
|
|
|
|
}
|
|
|
|
|
|
2011-07-28 09:57:45 -06:00
|
|
|
if (notifyNow) processCallbacks(List(func))
|
2011-02-28 10:17:44 -07:00
|
|
|
|
2010-11-12 12:11:53 +01:00
|
|
|
this
|
2009-05-25 14:48:43 +02:00
|
|
|
}
|
|
|
|
|
|
2011-06-17 06:51:40 -06:00
|
|
|
def onTimeout(func: Future[T] ⇒ Unit): this.type = {
|
2011-06-18 23:23:47 -06:00
|
|
|
if (timeout.duration.isFinite) {
|
|
|
|
|
_lock.lock
|
|
|
|
|
val runNow = try {
|
|
|
|
|
if (_value.isEmpty) {
|
|
|
|
|
if (!isExpired) {
|
|
|
|
|
val runnable = new Runnable {
|
|
|
|
|
def run() {
|
|
|
|
|
if (!isCompleted) func(self)
|
|
|
|
|
}
|
2011-06-17 06:51:40 -06:00
|
|
|
}
|
2011-06-18 23:23:47 -06:00
|
|
|
Scheduler.scheduleOnce(runnable, timeLeft, NANOS)
|
|
|
|
|
false
|
|
|
|
|
} else true
|
|
|
|
|
} else false
|
|
|
|
|
} finally {
|
|
|
|
|
_lock.unlock
|
|
|
|
|
}
|
2011-06-17 06:51:40 -06:00
|
|
|
|
2011-06-18 23:23:47 -06:00
|
|
|
if (runNow) func(this)
|
|
|
|
|
}
|
2011-06-17 06:51:40 -06:00
|
|
|
|
|
|
|
|
this
|
|
|
|
|
}
|
|
|
|
|
|
2011-06-20 16:22:46 -06:00
|
|
|
final def orElse[A >: T](fallback: ⇒ A): Future[A] =
|
|
|
|
|
if (timeout.duration.isFinite) {
|
|
|
|
|
value match {
|
|
|
|
|
case Some(_) ⇒ this
|
|
|
|
|
case _ if isExpired ⇒ new KeptPromise[A](try { Right(fallback) } catch { case e: Exception ⇒ Left(e) })
|
|
|
|
|
case _ ⇒
|
|
|
|
|
val promise = new DefaultPromise[A](Timeout.never)
|
|
|
|
|
promise completeWith this
|
|
|
|
|
val runnable = new Runnable {
|
|
|
|
|
def run() {
|
|
|
|
|
if (!isCompleted) promise complete (try { Right(fallback) } catch { case e: Exception ⇒ Left(e) })
|
|
|
|
|
}
|
|
|
|
|
}
|
|
|
|
|
Scheduler.scheduleOnce(runnable, timeLeft, NANOS)
|
|
|
|
|
promise
|
|
|
|
|
}
|
|
|
|
|
} else this
|
|
|
|
|
|
2011-05-18 17:25:30 +02:00
|
|
|
private def notifyCompleted(func: Future[T] ⇒ Unit) {
|
2011-03-18 17:26:53 +01:00
|
|
|
try {
|
|
|
|
|
func(this)
|
|
|
|
|
} catch {
|
2011-08-01 12:52:18 +02:00
|
|
|
case e ⇒ EventHandler.error(e, this, "Future onComplete-callback raised an exception")
|
2011-03-18 17:26:53 +01:00
|
|
|
}
|
2010-11-12 12:11:53 +01:00
|
|
|
}
|
2010-10-31 19:27:55 +01:00
|
|
|
|
2011-05-18 17:25:30 +02:00
|
|
|
@inline
|
|
|
|
|
private def currentTimeInNanos: Long = MILLIS.toNanos(System.currentTimeMillis)
|
|
|
|
|
@inline
|
|
|
|
|
private def timeLeft(): Long = timeoutInNanos - (currentTimeInNanos - _startTimeInNanos)
|
2009-05-25 14:48:43 +02:00
|
|
|
}
|
2011-03-18 17:26:53 +01:00
|
|
|
|
2011-08-02 10:19:49 -06:00
|
|
|
class ActorPromise(timeout: Timeout)(implicit dispatcher: MessageDispatcher) extends DefaultPromise[Any](timeout)(dispatcher) with ForwardableChannel {
|
2011-06-13 22:36:46 +02:00
|
|
|
|
|
|
|
|
def !(message: Any)(implicit channel: UntypedChannel = NullChannel) = completeWithResult(message)
|
|
|
|
|
|
|
|
|
|
def sendException(ex: Throwable) = completeWithException(ex)
|
|
|
|
|
|
|
|
|
|
def channel: UntypedChannel = this
|
|
|
|
|
|
|
|
|
|
def isUsableOnlyOnce = true
|
|
|
|
|
def isUsable = !isCompleted
|
|
|
|
|
def isReplyable = false
|
|
|
|
|
def canSendException = true
|
|
|
|
|
|
|
|
|
|
@deprecated("ActorPromise merged with Channel[Any], just use 'this'", "1.2")
|
|
|
|
|
def future = this
|
|
|
|
|
}
|
|
|
|
|
|
|
|
|
|
object ActorPromise {
|
2011-08-03 10:11:24 -06:00
|
|
|
def apply(f: Promise[Any]): ActorPromise =
|
|
|
|
|
new ActorPromise(f.timeout)(f.dispatcher) {
|
2011-06-13 22:36:46 +02:00
|
|
|
completeWith(f)
|
|
|
|
|
override def !(message: Any)(implicit channel: UntypedChannel) = f completeWithResult message
|
|
|
|
|
override def sendException(ex: Throwable) = f completeWithException ex
|
|
|
|
|
}
|
|
|
|
|
}
|
|
|
|
|
|
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-08-02 10:19:49 -06:00
|
|
|
sealed class KeptPromise[T](suppliedValue: Either[Throwable, T])(implicit val dispatcher: MessageDispatcher) extends Promise[T] {
|
2011-03-18 17:26:53 +01:00
|
|
|
val value = Some(suppliedValue)
|
|
|
|
|
|
2011-06-13 22:36:46 +02:00
|
|
|
def complete(value: Either[Throwable, T]): this.type = this
|
2011-08-02 10:19:49 -06:00
|
|
|
def onComplete(func: Future[T] ⇒ Unit): this.type = {
|
2011-07-28 10:34:08 -06:00
|
|
|
dispatcher dispatchTask (() ⇒ func(this)) //TODO: Use pending callback stack
|
2011-07-28 10:10:41 -06:00
|
|
|
this
|
|
|
|
|
}
|
2011-06-13 22:36:46 +02:00
|
|
|
def await(atMost: Duration): this.type = this
|
|
|
|
|
def await: this.type = this
|
2011-03-22 14:45:31 +01:00
|
|
|
def isExpired: Boolean = true
|
2011-06-18 23:23:47 -06:00
|
|
|
def timeout: Timeout = Timeout.zero
|
2011-05-19 14:37:54 -06:00
|
|
|
|
2011-06-17 06:51:40 -06:00
|
|
|
final def onTimeout(func: Future[T] ⇒ Unit): this.type = this
|
2011-06-20 16:22:46 -06:00
|
|
|
final def orElse[A >: T](fallback: ⇒ A): Future[A] = this
|
2011-06-17 06:51:40 -06:00
|
|
|
|
2011-03-18 17:26:53 +01:00
|
|
|
}
|