2015-01-28 14:19:50 +01:00
|
|
|
/**
|
|
|
|
|
* Copyright (C) 2014 Typesafe Inc. <http://www.typesafe.com>
|
|
|
|
|
*/
|
|
|
|
|
package akka.stream.impl
|
|
|
|
|
|
|
|
|
|
import java.util.concurrent.atomic.AtomicReference
|
|
|
|
|
|
|
|
|
|
import akka.actor.{ ActorRef, Props }
|
|
|
|
|
import akka.stream.impl.StreamLayout.Module
|
|
|
|
|
import akka.stream.scaladsl.OperationAttributes
|
|
|
|
|
import akka.stream.{ Inlet, Shape, SinkShape }
|
|
|
|
|
import org.reactivestreams.{ Publisher, Subscriber, Subscription }
|
|
|
|
|
|
|
|
|
|
import scala.annotation.unchecked.uncheckedVariance
|
|
|
|
|
import scala.concurrent.{ Future, Promise }
|
|
|
|
|
|
2015-03-05 12:21:17 +01:00
|
|
|
/**
|
|
|
|
|
* INTERNAL API
|
|
|
|
|
*/
|
|
|
|
|
private[akka] abstract class SinkModule[-In, Mat](val shape: SinkShape[In]) extends Module {
|
2015-01-28 14:19:50 +01:00
|
|
|
|
|
|
|
|
def create(materializer: ActorFlowMaterializerImpl, flowName: String): (Subscriber[In] @uncheckedVariance, Mat)
|
|
|
|
|
|
|
|
|
|
override def replaceShape(s: Shape): Module =
|
|
|
|
|
if (s == shape) this
|
|
|
|
|
else throw new UnsupportedOperationException("cannot replace the shape of a Sink, you need to wrap it in a Graph for that")
|
|
|
|
|
|
|
|
|
|
// This is okay since we the only caller of this method is right below.
|
|
|
|
|
protected def newInstance(s: SinkShape[In] @uncheckedVariance): SinkModule[In, Mat]
|
|
|
|
|
|
|
|
|
|
override def carbonCopy: Module = {
|
|
|
|
|
val in = new Inlet[In](shape.inlet.toString)
|
|
|
|
|
newInstance(SinkShape(in))
|
|
|
|
|
}
|
|
|
|
|
|
|
|
|
|
override def subModules: Set[Module] = Set.empty
|
2015-03-05 12:21:17 +01:00
|
|
|
|
|
|
|
|
def amendShape(attr: OperationAttributes): SinkShape[In] = {
|
|
|
|
|
attr.nameOption match {
|
|
|
|
|
case None ⇒ shape
|
|
|
|
|
case s: Some[String] if s == attributes.nameOption ⇒ shape
|
|
|
|
|
case Some(name) ⇒ shape.copy(inlet = new Inlet(name + ".in"))
|
|
|
|
|
}
|
|
|
|
|
}
|
2015-01-28 14:19:50 +01:00
|
|
|
}
|
|
|
|
|
|
|
|
|
|
/**
|
2015-03-05 12:21:17 +01:00
|
|
|
* INTERNAL API
|
2015-01-28 14:19:50 +01:00
|
|
|
* Holds the downstream-most [[org.reactivestreams.Publisher]] interface of the materialized flow.
|
|
|
|
|
* The stream will not have any subscribers attached at this point, which means that after prefetching
|
|
|
|
|
* elements to fill the internal buffers it will assert back-pressure until
|
|
|
|
|
* a subscriber connects and creates demand for elements to be emitted.
|
|
|
|
|
*/
|
2015-03-05 12:21:17 +01:00
|
|
|
private[akka] class PublisherSink[In](val attributes: OperationAttributes, shape: SinkShape[In]) extends SinkModule[In, Publisher[In]](shape) {
|
2015-01-28 14:19:50 +01:00
|
|
|
|
|
|
|
|
override def toString: String = "PublisherSink"
|
|
|
|
|
|
|
|
|
|
override def create(materializer: ActorFlowMaterializerImpl, flowName: String): (Subscriber[In], Publisher[In]) = {
|
|
|
|
|
val pub = new VirtualPublisher[In]
|
|
|
|
|
val sub = new VirtualSubscriber[In](pub)
|
|
|
|
|
(sub, pub)
|
|
|
|
|
}
|
|
|
|
|
|
|
|
|
|
override protected def newInstance(shape: SinkShape[In]): SinkModule[In, Publisher[In]] = new PublisherSink[In](attributes, shape)
|
2015-03-05 12:21:17 +01:00
|
|
|
override def withAttributes(attr: OperationAttributes): Module = new PublisherSink[In](attr, amendShape(attr))
|
2015-01-28 14:19:50 +01:00
|
|
|
}
|
|
|
|
|
|
2015-03-05 12:21:17 +01:00
|
|
|
/**
|
|
|
|
|
* INTERNAL API
|
|
|
|
|
*/
|
|
|
|
|
private[akka] final class FanoutPublisherSink[In](
|
2015-01-28 14:19:50 +01:00
|
|
|
initialBufferSize: Int,
|
|
|
|
|
maximumBufferSize: Int,
|
|
|
|
|
val attributes: OperationAttributes,
|
|
|
|
|
shape: SinkShape[In])
|
|
|
|
|
extends SinkModule[In, Publisher[In]](shape) {
|
|
|
|
|
|
|
|
|
|
override def create(materializer: ActorFlowMaterializerImpl, flowName: String): (Subscriber[In], Publisher[In]) = {
|
|
|
|
|
val fanoutActor = materializer.actorOf(
|
|
|
|
|
Props(new FanoutProcessorImpl(materializer.settings, initialBufferSize, maximumBufferSize)), s"$flowName-fanoutPublisher")
|
|
|
|
|
val fanoutProcessor = ActorProcessorFactory[In, In](fanoutActor)
|
|
|
|
|
(fanoutProcessor, fanoutProcessor)
|
|
|
|
|
}
|
|
|
|
|
|
|
|
|
|
override protected def newInstance(shape: SinkShape[In]): SinkModule[In, Publisher[In]] =
|
|
|
|
|
new FanoutPublisherSink[In](initialBufferSize, maximumBufferSize, attributes, shape)
|
|
|
|
|
|
|
|
|
|
override def withAttributes(attr: OperationAttributes): Module =
|
2015-03-05 12:21:17 +01:00
|
|
|
new FanoutPublisherSink[In](initialBufferSize, maximumBufferSize, attr, amendShape(attr))
|
2015-01-28 14:19:50 +01:00
|
|
|
}
|
|
|
|
|
|
2015-03-05 12:21:17 +01:00
|
|
|
/**
|
|
|
|
|
* INTERNAL API
|
|
|
|
|
*/
|
|
|
|
|
private[akka] object HeadSink {
|
|
|
|
|
class HeadSinkSubscriber[In](p: Promise[In]) extends Subscriber[In] {
|
2015-01-28 14:19:50 +01:00
|
|
|
private val sub = new AtomicReference[Subscription]
|
2015-03-03 10:57:25 +01:00
|
|
|
override def onSubscribe(s: Subscription): Unit = {
|
|
|
|
|
ReactiveStreamsCompliance.requireNonNullSubscription(s)
|
2015-01-28 14:19:50 +01:00
|
|
|
if (!sub.compareAndSet(null, s)) s.cancel()
|
|
|
|
|
else s.request(1)
|
2015-03-03 10:57:25 +01:00
|
|
|
}
|
|
|
|
|
|
|
|
|
|
override def onNext(elem: In): Unit = {
|
|
|
|
|
ReactiveStreamsCompliance.requireNonNullElement(elem)
|
|
|
|
|
p.trySuccess(elem)
|
|
|
|
|
sub.get.cancel()
|
|
|
|
|
}
|
|
|
|
|
|
|
|
|
|
override def onError(t: Throwable): Unit = {
|
|
|
|
|
ReactiveStreamsCompliance.requireNonNullException(t)
|
|
|
|
|
p.tryFailure(t)
|
|
|
|
|
}
|
2015-01-28 14:19:50 +01:00
|
|
|
|
|
|
|
|
override def onComplete(): Unit = p.tryFailure(new NoSuchElementException("empty stream"))
|
|
|
|
|
}
|
|
|
|
|
|
|
|
|
|
}
|
|
|
|
|
|
|
|
|
|
/**
|
2015-03-05 12:21:17 +01:00
|
|
|
* INTERNAL API
|
2015-01-28 14:19:50 +01:00
|
|
|
* Holds a [[scala.concurrent.Future]] that will be fulfilled with the first
|
|
|
|
|
* thing that is signaled to this stream, which can be either an element (after
|
|
|
|
|
* which the upstream subscription is canceled), an error condition (putting
|
|
|
|
|
* the Future into the corresponding failed state) or the end-of-stream
|
|
|
|
|
* (failing the Future with a NoSuchElementException).
|
|
|
|
|
*/
|
2015-03-05 12:21:17 +01:00
|
|
|
private[akka] class HeadSink[In](val attributes: OperationAttributes, shape: SinkShape[In]) extends SinkModule[In, Future[In]](shape) {
|
2015-01-28 14:19:50 +01:00
|
|
|
|
|
|
|
|
override def create(materializer: ActorFlowMaterializerImpl, flowName: String) = {
|
|
|
|
|
val p = Promise[In]()
|
|
|
|
|
val sub = new HeadSink.HeadSinkSubscriber[In](p)
|
|
|
|
|
(sub, p.future)
|
|
|
|
|
}
|
|
|
|
|
|
|
|
|
|
override protected def newInstance(shape: SinkShape[In]): SinkModule[In, Future[In]] = new HeadSink[In](attributes, shape)
|
2015-03-05 12:21:17 +01:00
|
|
|
override def withAttributes(attr: OperationAttributes): Module = new HeadSink[In](attr, amendShape(attr))
|
2015-01-28 14:19:50 +01:00
|
|
|
|
|
|
|
|
override def toString: String = "HeadSink"
|
|
|
|
|
}
|
|
|
|
|
|
|
|
|
|
/**
|
2015-03-05 12:21:17 +01:00
|
|
|
* INTERNAL API
|
2015-01-28 14:19:50 +01:00
|
|
|
* Attaches a subscriber to this stream which will just discard all received
|
|
|
|
|
* elements.
|
|
|
|
|
*/
|
2015-03-05 12:21:17 +01:00
|
|
|
private[akka] final class BlackholeSink(val attributes: OperationAttributes, shape: SinkShape[Any]) extends SinkModule[Any, Unit](shape) {
|
2015-01-28 14:19:50 +01:00
|
|
|
|
|
|
|
|
override def create(materializer: ActorFlowMaterializerImpl, flowName: String) =
|
|
|
|
|
(new BlackholeSubscriber[Any](materializer.settings.maxInputBufferSize), ())
|
|
|
|
|
|
|
|
|
|
override protected def newInstance(shape: SinkShape[Any]): SinkModule[Any, Unit] = new BlackholeSink(attributes, shape)
|
2015-03-05 12:21:17 +01:00
|
|
|
override def withAttributes(attr: OperationAttributes): Module = new BlackholeSink(attr, amendShape(attr))
|
2015-01-28 14:19:50 +01:00
|
|
|
}
|
|
|
|
|
|
|
|
|
|
/**
|
2015-03-05 12:21:17 +01:00
|
|
|
* INTERNAL API
|
2015-01-28 14:19:50 +01:00
|
|
|
* Attaches a subscriber to this stream.
|
|
|
|
|
*/
|
2015-03-05 12:21:17 +01:00
|
|
|
private[akka] final class SubscriberSink[In](subscriber: Subscriber[In], val attributes: OperationAttributes, shape: SinkShape[In]) extends SinkModule[In, Unit](shape) {
|
2015-01-28 14:19:50 +01:00
|
|
|
|
|
|
|
|
override def create(materializer: ActorFlowMaterializerImpl, flowName: String) = (subscriber, ())
|
|
|
|
|
|
|
|
|
|
override protected def newInstance(shape: SinkShape[In]): SinkModule[In, Unit] = new SubscriberSink[In](subscriber, attributes, shape)
|
2015-03-05 12:21:17 +01:00
|
|
|
override def withAttributes(attr: OperationAttributes): Module = new SubscriberSink[In](subscriber, attr, amendShape(attr))
|
2015-01-28 14:19:50 +01:00
|
|
|
}
|
|
|
|
|
|
|
|
|
|
/**
|
2015-03-05 12:21:17 +01:00
|
|
|
* INTERNAL API
|
2015-01-28 14:19:50 +01:00
|
|
|
* A sink that immediately cancels its upstream upon materialization.
|
|
|
|
|
*/
|
2015-03-05 12:21:17 +01:00
|
|
|
private[akka] final class CancelSink(val attributes: OperationAttributes, shape: SinkShape[Any]) extends SinkModule[Any, Unit](shape) {
|
2015-01-28 14:19:50 +01:00
|
|
|
|
|
|
|
|
override def create(materializer: ActorFlowMaterializerImpl, flowName: String): (Subscriber[Any], Unit) = {
|
|
|
|
|
val subscriber = new Subscriber[Any] {
|
|
|
|
|
override def onError(t: Throwable): Unit = ()
|
|
|
|
|
override def onSubscribe(s: Subscription): Unit = s.cancel()
|
|
|
|
|
override def onComplete(): Unit = ()
|
|
|
|
|
override def onNext(t: Any): Unit = ()
|
|
|
|
|
}
|
|
|
|
|
(subscriber, ())
|
|
|
|
|
}
|
|
|
|
|
|
|
|
|
|
override protected def newInstance(shape: SinkShape[Any]): SinkModule[Any, Unit] = new CancelSink(attributes, shape)
|
2015-03-05 12:21:17 +01:00
|
|
|
override def withAttributes(attr: OperationAttributes): Module = new CancelSink(attr, amendShape(attr))
|
2015-01-28 14:19:50 +01:00
|
|
|
}
|
|
|
|
|
|
|
|
|
|
/**
|
2015-03-05 12:21:17 +01:00
|
|
|
* INTERNAL API
|
2015-01-28 14:19:50 +01:00
|
|
|
* Creates and wraps an actor into [[org.reactivestreams.Subscriber]] from the given `props`,
|
|
|
|
|
* which should be [[akka.actor.Props]] for an [[akka.stream.actor.ActorSubscriber]].
|
|
|
|
|
*/
|
2015-03-05 12:21:17 +01:00
|
|
|
private[akka] final class PropsSink[In](props: Props, val attributes: OperationAttributes, shape: SinkShape[In]) extends SinkModule[In, ActorRef](shape) {
|
2015-01-28 14:19:50 +01:00
|
|
|
|
|
|
|
|
override def create(materializer: ActorFlowMaterializerImpl, flowName: String) = {
|
|
|
|
|
val subscriberRef = materializer.actorOf(props, name = s"$flowName-props")
|
|
|
|
|
(akka.stream.actor.ActorSubscriber[In](subscriberRef), subscriberRef)
|
|
|
|
|
}
|
|
|
|
|
|
|
|
|
|
override protected def newInstance(shape: SinkShape[In]): SinkModule[In, ActorRef] = new PropsSink[In](props, attributes, shape)
|
2015-03-05 12:21:17 +01:00
|
|
|
override def withAttributes(attr: OperationAttributes): Module = new PropsSink[In](props, attr, amendShape(attr))
|
2015-01-28 14:19:50 +01:00
|
|
|
}
|