2014-03-30 09:27:19 +02:00
|
|
|
/**
|
|
|
|
|
* Copyright (C) 2014 Typesafe Inc. <http://www.typesafe.com>
|
|
|
|
|
*/
|
|
|
|
|
package akka.stream.impl
|
|
|
|
|
|
|
|
|
|
import scala.annotation.tailrec
|
|
|
|
|
import scala.collection.immutable
|
2014-07-22 12:21:53 +02:00
|
|
|
import org.reactivestreams.{ Publisher, Subscriber, Processor }
|
2014-03-30 09:27:19 +02:00
|
|
|
import akka.actor.ActorRefFactory
|
2014-05-20 16:02:09 +02:00
|
|
|
import akka.stream.{ OverflowStrategy, MaterializerSettings, FlowMaterializer, Transformer }
|
2014-04-28 15:10:20 +02:00
|
|
|
import scala.util.Try
|
2014-04-29 15:16:05 +02:00
|
|
|
import scala.concurrent.Future
|
|
|
|
|
import scala.util.Success
|
|
|
|
|
import scala.util.Failure
|
2014-05-08 19:34:58 +02:00
|
|
|
import java.util.concurrent.atomic.AtomicLong
|
|
|
|
|
import akka.actor.ActorContext
|
|
|
|
|
import akka.actor.ExtensionIdProvider
|
|
|
|
|
import akka.actor.ExtensionId
|
|
|
|
|
import akka.actor.ExtendedActorSystem
|
|
|
|
|
import akka.actor.ActorSystem
|
|
|
|
|
import akka.actor.Extension
|
2014-05-22 20:58:38 +02:00
|
|
|
import scala.concurrent.duration.FiniteDuration
|
2014-05-20 13:46:35 +02:00
|
|
|
import akka.stream.TimerTransformer
|
2014-08-21 12:35:38 +02:00
|
|
|
import akka.actor.Props
|
|
|
|
|
import akka.actor.Actor
|
|
|
|
|
import akka.actor.ActorRef
|
|
|
|
|
import akka.pattern.ask
|
|
|
|
|
import akka.util.Timeout
|
|
|
|
|
import scala.concurrent.duration._
|
|
|
|
|
import scala.concurrent.Await
|
|
|
|
|
import akka.actor.LocalActorRef
|
|
|
|
|
import akka.actor.RepointableActorRef
|
|
|
|
|
import akka.actor.ActorCell
|
2014-03-30 09:27:19 +02:00
|
|
|
|
|
|
|
|
/**
|
|
|
|
|
* INTERNAL API
|
|
|
|
|
*/
|
|
|
|
|
private[akka] object Ast {
|
2014-05-22 08:40:41 +02:00
|
|
|
sealed trait AstNode {
|
2014-05-08 19:34:58 +02:00
|
|
|
def name: String
|
|
|
|
|
}
|
2014-03-30 09:27:19 +02:00
|
|
|
|
2014-05-08 19:34:58 +02:00
|
|
|
case class Transform(transformer: Transformer[Any, Any]) extends AstNode {
|
|
|
|
|
override def name = transformer.name
|
|
|
|
|
}
|
2014-05-23 13:52:39 +02:00
|
|
|
case class MapFuture(f: Any ⇒ Future[Any]) extends AstNode {
|
|
|
|
|
override def name = "mapFuture"
|
|
|
|
|
}
|
2014-05-08 19:34:58 +02:00
|
|
|
case class GroupBy(f: Any ⇒ Any) extends AstNode {
|
|
|
|
|
override def name = "groupBy"
|
|
|
|
|
}
|
|
|
|
|
case class SplitWhen(p: Any ⇒ Boolean) extends AstNode {
|
|
|
|
|
override def name = "splitWhen"
|
|
|
|
|
}
|
2014-07-22 12:21:53 +02:00
|
|
|
case class Merge(other: Publisher[Any]) extends AstNode {
|
2014-05-08 19:34:58 +02:00
|
|
|
override def name = "merge"
|
|
|
|
|
}
|
2014-07-22 12:21:53 +02:00
|
|
|
case class Zip(other: Publisher[Any]) extends AstNode {
|
2014-05-08 19:34:58 +02:00
|
|
|
override def name = "zip"
|
|
|
|
|
}
|
2014-07-22 12:21:53 +02:00
|
|
|
case class Concat(next: Publisher[Any]) extends AstNode {
|
2014-05-08 19:34:58 +02:00
|
|
|
override def name = "concat"
|
|
|
|
|
}
|
2014-07-17 14:48:01 +02:00
|
|
|
case class Broadcast(other: Subscriber[Any]) extends AstNode {
|
|
|
|
|
override def name = "broadcast"
|
2014-05-08 19:34:58 +02:00
|
|
|
}
|
2014-05-16 14:21:15 +02:00
|
|
|
case class PrefixAndTail(n: Int) extends AstNode {
|
|
|
|
|
override def name = "prefixAndTail"
|
|
|
|
|
}
|
|
|
|
|
case object ConcatAll extends AstNode {
|
|
|
|
|
override def name = "concatFlatten"
|
|
|
|
|
}
|
2014-03-28 15:44:18 +01:00
|
|
|
|
2014-05-20 16:02:09 +02:00
|
|
|
case class Conflate(seed: Any ⇒ Any, aggregate: (Any, Any) ⇒ Any) extends AstNode {
|
|
|
|
|
override def name = "conflate"
|
|
|
|
|
}
|
|
|
|
|
|
|
|
|
|
case class Expand(seed: Any ⇒ Any, extrapolate: Any ⇒ (Any, Any)) extends AstNode {
|
|
|
|
|
override def name = "expand"
|
|
|
|
|
}
|
|
|
|
|
|
|
|
|
|
case class Buffer(size: Int, overflowStrategy: OverflowStrategy) extends AstNode {
|
|
|
|
|
override def name = "buffer"
|
|
|
|
|
}
|
|
|
|
|
|
2014-07-22 12:21:53 +02:00
|
|
|
trait PublisherNode[I] {
|
|
|
|
|
private[akka] def createPublisher(materializer: ActorBasedFlowMaterializer, flowName: String): Publisher[I]
|
2014-03-28 15:44:18 +01:00
|
|
|
}
|
|
|
|
|
|
2014-07-22 12:21:53 +02:00
|
|
|
final case class ExistingPublisher[I](publisher: Publisher[I]) extends PublisherNode[I] {
|
|
|
|
|
def createPublisher(materializer: ActorBasedFlowMaterializer, flowName: String) = publisher
|
2014-03-28 15:44:18 +01:00
|
|
|
}
|
|
|
|
|
|
2014-07-22 12:21:53 +02:00
|
|
|
final case class IteratorPublisherNode[I](iterator: Iterator[I]) extends PublisherNode[I] {
|
|
|
|
|
final def createPublisher(materializer: ActorBasedFlowMaterializer, flowName: String): Publisher[I] =
|
|
|
|
|
if (iterator.isEmpty) EmptyPublisher.asInstanceOf[Publisher[I]]
|
2014-08-21 12:35:38 +02:00
|
|
|
else ActorPublisher[I](materializer.actorOf(IteratorPublisher.props(iterator, materializer.settings),
|
2014-05-08 19:34:58 +02:00
|
|
|
name = s"$flowName-0-iterator"))
|
2014-03-28 15:44:18 +01:00
|
|
|
}
|
2014-07-22 12:21:53 +02:00
|
|
|
final case class IterablePublisherNode[I](iterable: immutable.Iterable[I]) extends PublisherNode[I] {
|
|
|
|
|
def createPublisher(materializer: ActorBasedFlowMaterializer, flowName: String): Publisher[I] =
|
|
|
|
|
if (iterable.isEmpty) EmptyPublisher.asInstanceOf[Publisher[I]]
|
2014-08-21 12:35:38 +02:00
|
|
|
else ActorPublisher[I](materializer.actorOf(IterablePublisher.props(iterable, materializer.settings),
|
2014-05-08 11:25:47 +02:00
|
|
|
name = s"$flowName-0-iterable"), Some(iterable))
|
2014-03-28 15:44:18 +01:00
|
|
|
}
|
2014-07-22 12:21:53 +02:00
|
|
|
final case class ThunkPublisherNode[I](f: () ⇒ I) extends PublisherNode[I] {
|
|
|
|
|
def createPublisher(materializer: ActorBasedFlowMaterializer, flowName: String): Publisher[I] =
|
2014-08-21 12:35:38 +02:00
|
|
|
ActorPublisher[I](materializer.actorOf(SimpleCallbackPublisher.props(materializer.settings, f),
|
2014-05-08 19:34:58 +02:00
|
|
|
name = s"$flowName-0-thunk"))
|
2014-04-02 08:07:05 +02:00
|
|
|
}
|
2014-07-22 12:21:53 +02:00
|
|
|
final case class FuturePublisherNode[I](future: Future[I]) extends PublisherNode[I] {
|
|
|
|
|
def createPublisher(materializer: ActorBasedFlowMaterializer, flowName: String): Publisher[I] =
|
2014-04-29 15:16:05 +02:00
|
|
|
future.value match {
|
|
|
|
|
case Some(Success(element)) ⇒
|
2014-08-21 12:35:38 +02:00
|
|
|
ActorPublisher[I](materializer.actorOf(IterablePublisher.props(List(element), materializer.settings),
|
2014-05-08 11:25:47 +02:00
|
|
|
name = s"$flowName-0-future"), Some(future))
|
2014-04-29 15:16:05 +02:00
|
|
|
case Some(Failure(t)) ⇒
|
2014-07-22 12:21:53 +02:00
|
|
|
ErrorPublisher(t).asInstanceOf[Publisher[I]]
|
2014-04-29 15:16:05 +02:00
|
|
|
case None ⇒
|
2014-08-21 12:35:38 +02:00
|
|
|
ActorPublisher[I](materializer.actorOf(FuturePublisher.props(future, materializer.settings),
|
2014-05-08 11:25:47 +02:00
|
|
|
name = s"$flowName-0-future"), Some(future))
|
2014-04-29 15:16:05 +02:00
|
|
|
}
|
|
|
|
|
}
|
2014-07-18 10:55:34 +07:00
|
|
|
final case class TickPublisherNode[I](initialDelay: FiniteDuration, interval: FiniteDuration, tick: () ⇒ I) extends PublisherNode[I] {
|
2014-07-22 12:21:53 +02:00
|
|
|
def createPublisher(materializer: ActorBasedFlowMaterializer, flowName: String): Publisher[I] =
|
2014-08-21 12:35:38 +02:00
|
|
|
ActorPublisher[I](materializer.actorOf(TickPublisher.props(initialDelay, interval, tick, materializer.settings),
|
2014-05-22 20:58:38 +02:00
|
|
|
name = s"$flowName-0-tick"))
|
|
|
|
|
}
|
2014-03-30 09:27:19 +02:00
|
|
|
}
|
|
|
|
|
|
|
|
|
|
/**
|
|
|
|
|
* INTERNAL API
|
|
|
|
|
*/
|
2014-08-21 12:35:38 +02:00
|
|
|
private[akka] case class ActorBasedFlowMaterializer(
|
|
|
|
|
override val settings: MaterializerSettings,
|
|
|
|
|
supervisor: ActorRef,
|
|
|
|
|
flowNameCounter: AtomicLong,
|
2014-05-08 19:34:58 +02:00
|
|
|
namePrefix: String)
|
2014-05-22 08:40:41 +02:00
|
|
|
extends FlowMaterializer(settings) {
|
2014-03-30 09:27:19 +02:00
|
|
|
import Ast._
|
2014-04-08 13:37:55 +02:00
|
|
|
import ActorBasedFlowMaterializer._
|
2014-04-02 09:03:59 +02:00
|
|
|
|
2014-08-21 12:35:38 +02:00
|
|
|
def withNamePrefix(name: String): FlowMaterializer = this.copy(namePrefix = name)
|
2014-05-08 19:34:58 +02:00
|
|
|
|
2014-08-21 12:35:38 +02:00
|
|
|
private def nextFlowNameCount(): Long = flowNameCounter.incrementAndGet()
|
2014-05-08 19:34:58 +02:00
|
|
|
|
|
|
|
|
private def createFlowName(): String = s"$namePrefix-${nextFlowNameCount()}"
|
|
|
|
|
|
2014-07-22 12:21:53 +02:00
|
|
|
@tailrec private def processorChain(topSubscriber: Subscriber[_], ops: immutable.Seq[AstNode],
|
|
|
|
|
flowName: String, n: Int): Subscriber[_] = {
|
2014-03-30 09:27:19 +02:00
|
|
|
ops match {
|
|
|
|
|
case op :: tail ⇒
|
2014-05-08 19:34:58 +02:00
|
|
|
val opProcessor: Processor[Any, Any] = processorForNode(op, flowName, n)
|
2014-07-22 12:21:53 +02:00
|
|
|
opProcessor.subscribe(topSubscriber.asInstanceOf[Subscriber[Any]])
|
2014-05-08 19:34:58 +02:00
|
|
|
processorChain(opProcessor, tail, flowName, n - 1)
|
2014-07-22 12:21:53 +02:00
|
|
|
case _ ⇒ topSubscriber
|
2014-03-30 09:27:19 +02:00
|
|
|
}
|
|
|
|
|
}
|
|
|
|
|
|
|
|
|
|
// Ops come in reverse order
|
2014-07-22 12:21:53 +02:00
|
|
|
override def toPublisher[I, O](publisherNode: PublisherNode[I], ops: List[AstNode]): Publisher[O] = {
|
2014-05-08 19:34:58 +02:00
|
|
|
val flowName = createFlowName()
|
2014-07-22 12:21:53 +02:00
|
|
|
if (ops.isEmpty) publisherNode.createPublisher(this, flowName).asInstanceOf[Publisher[O]]
|
2014-03-30 09:27:19 +02:00
|
|
|
else {
|
2014-05-08 19:34:58 +02:00
|
|
|
val opsSize = ops.size
|
|
|
|
|
val opProcessor = processorForNode(ops.head, flowName, opsSize)
|
2014-07-22 12:21:53 +02:00
|
|
|
val topSubscriber = processorChain(opProcessor, ops.tail, flowName, opsSize - 1)
|
|
|
|
|
publisherNode.createPublisher(this, flowName).subscribe(topSubscriber.asInstanceOf[Subscriber[I]])
|
|
|
|
|
opProcessor.asInstanceOf[Publisher[O]]
|
2014-03-30 09:27:19 +02:00
|
|
|
}
|
|
|
|
|
}
|
|
|
|
|
|
2014-05-07 15:56:02 +02:00
|
|
|
private val identityTransform = Transform(
|
|
|
|
|
new Transformer[Any, Any] {
|
|
|
|
|
override def onNext(element: Any) = List(element)
|
|
|
|
|
})
|
|
|
|
|
|
2014-08-21 12:35:38 +02:00
|
|
|
def processorForNode(op: AstNode, flowName: String, n: Int): Processor[Any, Any] = {
|
|
|
|
|
val impl = actorOf(ActorProcessor.props(settings, op), s"$flowName-$n-${op.name}")
|
|
|
|
|
ActorProcessor(impl)
|
|
|
|
|
|
|
|
|
|
}
|
|
|
|
|
|
|
|
|
|
def actorOf(props: Props, name: String): ActorRef = supervisor match {
|
|
|
|
|
case ref: LocalActorRef ⇒
|
|
|
|
|
ref.underlying.attachChild(props, name, systemService = false)
|
|
|
|
|
case ref: RepointableActorRef ⇒
|
|
|
|
|
if (ref.isStarted)
|
|
|
|
|
ref.underlying.asInstanceOf[ActorCell].attachChild(props, name, systemService = false)
|
|
|
|
|
else {
|
|
|
|
|
implicit val timeout = ref.system.settings.CreationTimeout
|
|
|
|
|
val f = (supervisor ? StreamSupervisor.Materialize(props, name)).mapTo[ActorRef]
|
|
|
|
|
Await.result(f, timeout.duration)
|
|
|
|
|
}
|
|
|
|
|
case _ ⇒
|
|
|
|
|
throw new IllegalStateException(s"Stream supervisor must be a local actor, was [${supervisor.getClass.getName}]")
|
|
|
|
|
}
|
2014-03-30 09:27:19 +02:00
|
|
|
|
2014-07-22 12:21:53 +02:00
|
|
|
override def ductProduceTo[In, Out](subscriber: Subscriber[Out], ops: List[Ast.AstNode]): Subscriber[In] =
|
|
|
|
|
processorChain(subscriber, ops, createFlowName(), ops.size).asInstanceOf[Subscriber[In]]
|
2014-05-07 15:56:02 +02:00
|
|
|
|
2014-07-22 12:21:53 +02:00
|
|
|
override def ductBuild[In, Out](ops: List[Ast.AstNode]): (Subscriber[In], Publisher[Out]) = {
|
2014-05-08 19:34:58 +02:00
|
|
|
val flowName = createFlowName()
|
2014-05-07 15:56:02 +02:00
|
|
|
if (ops.isEmpty) {
|
2014-05-08 19:34:58 +02:00
|
|
|
val identityProcessor: Processor[In, Out] = processorForNode(identityTransform, flowName, 1).asInstanceOf[Processor[In, Out]]
|
2014-05-07 15:56:02 +02:00
|
|
|
(identityProcessor, identityProcessor)
|
|
|
|
|
} else {
|
2014-05-08 19:34:58 +02:00
|
|
|
val opsSize = ops.size
|
|
|
|
|
val outProcessor = processorForNode(ops.head, flowName, opsSize).asInstanceOf[Processor[In, Out]]
|
2014-07-22 12:21:53 +02:00
|
|
|
val topSubscriber = processorChain(outProcessor, ops.tail, flowName, opsSize - 1).asInstanceOf[Processor[In, Out]]
|
|
|
|
|
(topSubscriber, outProcessor)
|
2014-05-07 15:56:02 +02:00
|
|
|
}
|
|
|
|
|
}
|
|
|
|
|
|
2014-03-30 09:27:19 +02:00
|
|
|
}
|
2014-05-08 19:34:58 +02:00
|
|
|
|
|
|
|
|
/**
|
|
|
|
|
* INTERNAL API
|
|
|
|
|
*/
|
|
|
|
|
private[akka] object FlowNameCounter extends ExtensionId[FlowNameCounter] with ExtensionIdProvider {
|
|
|
|
|
override def get(system: ActorSystem): FlowNameCounter = super.get(system)
|
|
|
|
|
override def lookup = FlowNameCounter
|
|
|
|
|
override def createExtension(system: ExtendedActorSystem): FlowNameCounter = new FlowNameCounter
|
|
|
|
|
}
|
|
|
|
|
|
|
|
|
|
/**
|
|
|
|
|
* INTERNAL API
|
|
|
|
|
*/
|
|
|
|
|
private[akka] class FlowNameCounter extends Extension {
|
|
|
|
|
val counter = new AtomicLong(0)
|
2014-08-21 12:35:38 +02:00
|
|
|
}
|
|
|
|
|
|
|
|
|
|
/**
|
|
|
|
|
* INTERNAL API
|
|
|
|
|
*/
|
|
|
|
|
private[akka] object StreamSupervisor {
|
|
|
|
|
def props(settings: MaterializerSettings): Props = Props(new StreamSupervisor(settings))
|
|
|
|
|
|
|
|
|
|
case class Materialize(props: Props, name: String)
|
|
|
|
|
}
|
|
|
|
|
|
|
|
|
|
private[akka] class StreamSupervisor(settings: MaterializerSettings) extends Actor {
|
|
|
|
|
import StreamSupervisor._
|
|
|
|
|
|
|
|
|
|
def receive = {
|
|
|
|
|
case Materialize(props, name) ⇒
|
|
|
|
|
val impl = context.actorOf(props, name)
|
|
|
|
|
sender() ! impl
|
|
|
|
|
}
|
2014-05-08 19:34:58 +02:00
|
|
|
}
|