2014-09-03 21:54:18 +02:00
|
|
|
/**
|
2015-04-16 02:24:01 +02:00
|
|
|
* Copyright (C) 2014-2015 Typesafe Inc. <http://www.typesafe.com>
|
2014-09-03 21:54:18 +02:00
|
|
|
*/
|
2014-10-27 14:35:41 +01:00
|
|
|
package akka.stream.scaladsl
|
2014-09-03 21:54:18 +02:00
|
|
|
|
2015-04-16 02:24:01 +02:00
|
|
|
import akka.actor.{ ActorRef, Cancellable, Props }
|
2015-10-09 15:11:01 -04:00
|
|
|
import akka.stream.actor.ActorPublisher
|
2015-06-29 23:47:31 -04:00
|
|
|
import akka.stream.impl.Stages.{ DefaultAttributes, MaterializingStageFactory, StageModule }
|
2015-07-06 22:00:21 +02:00
|
|
|
import akka.stream.impl.StreamLayout.Module
|
2015-09-18 14:30:43 +02:00
|
|
|
import akka.stream.impl.fusing.GraphStages.TickSource
|
2015-06-29 23:47:31 -04:00
|
|
|
import akka.stream.impl.{ EmptyPublisher, ErrorPublisher, _ }
|
|
|
|
|
import akka.stream.{ Outlet, SourceShape, _ }
|
2015-07-06 22:00:21 +02:00
|
|
|
import org.reactivestreams.{ Publisher, Subscriber }
|
2015-06-29 23:47:31 -04:00
|
|
|
|
|
|
|
|
import scala.annotation.tailrec
|
2014-09-03 21:54:18 +02:00
|
|
|
import scala.collection.immutable
|
2015-08-19 23:04:20 -04:00
|
|
|
import scala.concurrent.duration.{ FiniteDuration, _ }
|
2015-04-16 02:24:01 +02:00
|
|
|
import scala.concurrent.{ Future, Promise }
|
2015-06-29 23:47:31 -04:00
|
|
|
import scala.language.higherKinds
|
2014-09-03 21:54:18 +02:00
|
|
|
|
2014-10-02 17:32:08 +02:00
|
|
|
/**
|
2015-01-28 14:19:50 +01:00
|
|
|
* A `Source` is a set of stream processing steps that has one open output. It can comprise
|
|
|
|
|
* any number of internal sources and transformations that are wired together, or it can be
|
|
|
|
|
* an “atomic” source, e.g. from a collection or a file. Materialization turns a Source into
|
|
|
|
|
* a Reactive Streams `Publisher` (at least conceptually).
|
2014-10-02 17:32:08 +02:00
|
|
|
*/
|
2015-01-28 14:19:50 +01:00
|
|
|
final class Source[+Out, +Mat](private[stream] override val module: Module)
|
|
|
|
|
extends FlowOps[Out, Mat] with Graph[SourceShape[Out], Mat] {
|
|
|
|
|
|
|
|
|
|
override type Repr[+O, +M] = Source[O, M]
|
|
|
|
|
|
|
|
|
|
override val shape: SourceShape[Out] = module.shape.asInstanceOf[SourceShape[Out]]
|
|
|
|
|
|
2015-04-24 12:14:04 +02:00
|
|
|
def viaMat[T, Mat2, Mat3](flow: Graph[FlowShape[Out, T], Mat2])(combine: (Mat, Mat2) ⇒ Mat3): Source[T, Mat3] = {
|
|
|
|
|
if (flow.module.isInstanceOf[Stages.Identity]) this.asInstanceOf[Source[T, Mat3]]
|
2015-01-28 14:19:50 +01:00
|
|
|
else {
|
|
|
|
|
val flowCopy = flow.module.carbonCopy
|
|
|
|
|
new Source(
|
|
|
|
|
module
|
2015-07-06 22:00:21 +02:00
|
|
|
.fuse(flowCopy, shape.outlet, flowCopy.shape.inlets.head, combine)
|
2015-09-17 10:08:12 +02:00
|
|
|
.replaceShape(SourceShape(flowCopy.shape.outlets.head)))
|
2015-01-28 14:19:50 +01:00
|
|
|
}
|
|
|
|
|
}
|
2014-10-02 17:32:08 +02:00
|
|
|
|
|
|
|
|
/**
|
2014-11-06 14:03:01 +01:00
|
|
|
* Connect this [[akka.stream.scaladsl.Source]] to a [[akka.stream.scaladsl.Sink]],
|
|
|
|
|
* concatenating the processing steps of both.
|
2014-10-02 17:32:08 +02:00
|
|
|
*/
|
2015-06-23 18:41:55 +02:00
|
|
|
def to[Mat2](sink: Graph[SinkShape[Out], Mat2]): RunnableGraph[Mat] = toMat(sink)(Keep.left)
|
2015-01-28 14:19:50 +01:00
|
|
|
|
|
|
|
|
/**
|
|
|
|
|
* Connect this [[akka.stream.scaladsl.Source]] to a [[akka.stream.scaladsl.Sink]],
|
|
|
|
|
* concatenating the processing steps of both.
|
|
|
|
|
*/
|
2015-06-23 18:41:55 +02:00
|
|
|
def toMat[Mat2, Mat3](sink: Graph[SinkShape[Out], Mat2])(combine: (Mat, Mat2) ⇒ Mat3): RunnableGraph[Mat3] = {
|
2015-01-28 14:19:50 +01:00
|
|
|
val sinkCopy = sink.module.carbonCopy
|
2015-07-06 22:00:21 +02:00
|
|
|
RunnableGraph(module.fuse(sinkCopy, shape.outlet, sinkCopy.shape.inlets.head, combine))
|
2015-01-28 14:19:50 +01:00
|
|
|
}
|
|
|
|
|
|
|
|
|
|
/**
|
|
|
|
|
* Transform only the materialized value of this Source, leaving all other properties as they were.
|
|
|
|
|
*/
|
2015-05-05 10:29:41 +02:00
|
|
|
def mapMaterializedValue[Mat2](f: Mat ⇒ Mat2): Repr[Out, Mat2] =
|
2015-01-28 14:19:50 +01:00
|
|
|
new Source(module.transformMaterializedValue(f.asInstanceOf[Any ⇒ Any]))
|
|
|
|
|
|
|
|
|
|
/** INTERNAL API */
|
|
|
|
|
override private[scaladsl] def andThen[U](op: StageModule): Repr[U, Mat] = {
|
|
|
|
|
// No need to copy here, op is a fresh instance
|
|
|
|
|
new Source(
|
|
|
|
|
module
|
2015-07-06 22:00:21 +02:00
|
|
|
.fuse(op, shape.outlet, op.inPort)
|
2015-01-28 14:19:50 +01:00
|
|
|
.replaceShape(SourceShape(op.outPort)))
|
|
|
|
|
}
|
|
|
|
|
|
|
|
|
|
override private[scaladsl] def andThenMat[U, Mat2](op: MaterializingStageFactory): Repr[U, Mat2] = {
|
|
|
|
|
new Source(
|
|
|
|
|
module
|
2015-07-06 22:00:21 +02:00
|
|
|
.fuse(op, shape.outlet, op.inPort, Keep.right)
|
2015-01-28 14:19:50 +01:00
|
|
|
.replaceShape(SourceShape(op.outPort)))
|
|
|
|
|
}
|
2014-10-02 17:32:08 +02:00
|
|
|
|
2014-10-02 13:34:27 +02:00
|
|
|
/**
|
2014-10-17 14:05:50 +02:00
|
|
|
* Connect this `Source` to a `Sink` and run it. The returned value is the materialized value
|
2014-11-06 14:03:01 +01:00
|
|
|
* of the `Sink`, e.g. the `Publisher` of a [[akka.stream.scaladsl.Sink#publisher]].
|
2014-10-02 13:34:27 +02:00
|
|
|
*/
|
2015-06-23 18:28:53 +02:00
|
|
|
def runWith[Mat2](sink: Graph[SinkShape[Out], Mat2])(implicit materializer: Materializer): Mat2 = toMat(sink)(Keep.right).run()
|
2014-09-03 21:54:18 +02:00
|
|
|
|
2014-10-06 14:46:52 +02:00
|
|
|
/**
|
|
|
|
|
* Shortcut for running this `Source` with a fold function.
|
|
|
|
|
* The given function is invoked for every received element, giving it its previous
|
|
|
|
|
* output (or the given `zero` value) and the element as input.
|
|
|
|
|
* The returned [[scala.concurrent.Future]] will be completed with value of the final
|
|
|
|
|
* function evaluation when the input stream ends, or completed with `Failure`
|
2015-01-30 10:30:56 +01:00
|
|
|
* if there is a failure signaled in the stream.
|
2014-10-06 14:46:52 +02:00
|
|
|
*/
|
2015-06-23 18:28:53 +02:00
|
|
|
def runFold[U](zero: U)(f: (U, Out) ⇒ U)(implicit materializer: Materializer): Future[U] =
|
2015-01-28 14:19:50 +01:00
|
|
|
runWith(Sink.fold(zero)(f))
|
2014-10-06 14:46:52 +02:00
|
|
|
|
|
|
|
|
/**
|
|
|
|
|
* Shortcut for running this `Source` with a foreach procedure. The given procedure is invoked
|
|
|
|
|
* for each received element.
|
|
|
|
|
* The returned [[scala.concurrent.Future]] will be completed with `Success` when reaching the
|
2015-01-30 10:30:56 +01:00
|
|
|
* normal end of the stream, or completed with `Failure` if there is a failure signaled in
|
2014-10-06 14:46:52 +02:00
|
|
|
* the stream.
|
|
|
|
|
*/
|
2015-06-23 18:28:53 +02:00
|
|
|
def runForeach(f: Out ⇒ Unit)(implicit materializer: Materializer): Future[Unit] = runWith(Sink.foreach(f))
|
2014-10-06 14:46:52 +02:00
|
|
|
|
2015-07-06 22:00:21 +02:00
|
|
|
/**
|
|
|
|
|
* Nests the current Source and returns a Source with the given Attributes
|
|
|
|
|
* @param attr the attributes to add
|
|
|
|
|
* @return a new Source with the added attributes
|
|
|
|
|
*/
|
2015-06-23 17:32:55 +02:00
|
|
|
override def withAttributes(attr: Attributes): Repr[Out, Mat] =
|
2015-07-06 22:00:21 +02:00
|
|
|
new Source(module.withAttributes(attr).nest()) // User API
|
2014-12-01 20:07:55 +02:00
|
|
|
|
2015-06-23 17:32:55 +02:00
|
|
|
override def named(name: String): Repr[Out, Mat] = withAttributes(Attributes.name(name))
|
2015-04-14 08:59:37 +02:00
|
|
|
|
2015-03-06 12:22:14 +01:00
|
|
|
/** Converts this Scala DSL element to it's Java DSL counterpart. */
|
|
|
|
|
def asJava: javadsl.Source[Out, Mat] = new javadsl.Source(this)
|
2015-06-29 23:47:31 -04:00
|
|
|
|
|
|
|
|
/**
|
|
|
|
|
* Combines several sources with fun-in strategy like `Merge` or `Concat` and returns `Source`.
|
|
|
|
|
*/
|
|
|
|
|
def combine[T, U](first: Source[T, _], second: Source[T, _], rest: Source[T, _]*)(strategy: Int ⇒ Graph[UniformFanInShape[T, U], Unit]): Source[U, Unit] =
|
|
|
|
|
Source.wrap(FlowGraph.partial() { implicit b ⇒
|
|
|
|
|
import FlowGraph.Implicits._
|
|
|
|
|
val c = b.add(strategy(rest.size + 2))
|
|
|
|
|
first ~> c.in(0)
|
|
|
|
|
second ~> c.in(1)
|
|
|
|
|
|
|
|
|
|
@tailrec def combineRest(idx: Int, i: Iterator[Source[T, _]]): SourceShape[U] =
|
|
|
|
|
if (i.hasNext) {
|
|
|
|
|
i.next() ~> c.in(idx)
|
|
|
|
|
combineRest(idx + 1, i)
|
|
|
|
|
} else SourceShape(c.out)
|
|
|
|
|
|
|
|
|
|
combineRest(2, rest.iterator)
|
|
|
|
|
})
|
2014-10-02 17:32:08 +02:00
|
|
|
}
|
|
|
|
|
|
2015-01-28 14:19:50 +01:00
|
|
|
object Source extends SourceApply {
|
|
|
|
|
|
2015-06-06 16:45:01 +02:00
|
|
|
private[this] final val _id: Any ⇒ Any = x ⇒ x
|
|
|
|
|
private[this] final def id[A]: A ⇒ A = _id.asInstanceOf[A ⇒ A]
|
|
|
|
|
|
2015-01-28 14:19:50 +01:00
|
|
|
private[stream] def apply[Out, Mat](module: SourceModule[Out, Mat]): Source[Out, Mat] =
|
|
|
|
|
new Source(module)
|
|
|
|
|
|
2015-04-16 02:24:01 +02:00
|
|
|
/** INTERNAL API */
|
2015-06-13 16:28:38 -04:00
|
|
|
private[stream] def shape[T](name: String): SourceShape[T] = SourceShape(Outlet(name + ".out"))
|
2015-01-28 14:19:50 +01:00
|
|
|
|
|
|
|
|
/**
|
|
|
|
|
* Helper to create [[Source]] from `Publisher`.
|
|
|
|
|
*
|
|
|
|
|
* Construct a transformation starting with given publisher. The transformation steps
|
|
|
|
|
* are executed by a series of [[org.reactivestreams.Processor]] instances
|
|
|
|
|
* that mediate the flow of elements downstream and the propagation of
|
|
|
|
|
* back-pressure upstream.
|
|
|
|
|
*/
|
|
|
|
|
def apply[T](publisher: Publisher[T]): Source[T, Unit] =
|
2015-04-20 21:04:03 +02:00
|
|
|
new Source(new PublisherSource(publisher, DefaultAttributes.publisherSource, shape("PublisherSource")))
|
2015-01-28 14:19:50 +01:00
|
|
|
|
2014-09-03 21:54:18 +02:00
|
|
|
/**
|
2014-10-02 17:32:08 +02:00
|
|
|
* Helper to create [[Source]] from `Iterator`.
|
2014-11-09 21:09:50 +01:00
|
|
|
* Example usage: `Source(() => Iterator.from(0))`
|
2014-09-03 21:54:18 +02:00
|
|
|
*
|
2014-11-09 21:09:50 +01:00
|
|
|
* Start a new `Source` from the given function that produces anIterator.
|
|
|
|
|
* The produced stream of elements will continue until the iterator runs empty
|
|
|
|
|
* or fails during evaluation of the `next()` method.
|
|
|
|
|
* Elements are pulled out of the iterator in accordance with the demand coming
|
|
|
|
|
* from the downstream transformation steps.
|
2014-09-03 21:54:18 +02:00
|
|
|
*/
|
2015-06-06 16:07:35 +02:00
|
|
|
def apply[T](f: () ⇒ Iterator[T]): Source[T, Unit] =
|
2015-07-06 22:00:21 +02:00
|
|
|
apply(new immutable.Iterable[T] {
|
|
|
|
|
override def iterator: Iterator[T] = f()
|
|
|
|
|
override def toString: String = "() => Iterator"
|
|
|
|
|
})
|
2015-01-28 14:19:50 +01:00
|
|
|
|
|
|
|
|
/**
|
|
|
|
|
* A graph with the shape of a source logically is a source, this method makes
|
|
|
|
|
* it so also in type.
|
|
|
|
|
*/
|
2015-06-06 17:17:23 +02:00
|
|
|
def wrap[T, M](g: Graph[SourceShape[T], M]): Source[T, M] = g match {
|
|
|
|
|
case s: Source[T, M] ⇒ s
|
|
|
|
|
case other ⇒ new Source(other.module)
|
|
|
|
|
}
|
2014-09-03 21:54:18 +02:00
|
|
|
|
|
|
|
|
/**
|
2014-10-02 17:32:08 +02:00
|
|
|
* Helper to create [[Source]] from `Iterable`.
|
|
|
|
|
* Example usage: `Source(Seq(1,2,3))`
|
2014-09-03 21:54:18 +02:00
|
|
|
*
|
2014-10-02 17:32:08 +02:00
|
|
|
* Starts a new `Source` from the given `Iterable`. This is like starting from an
|
2014-09-03 21:54:18 +02:00
|
|
|
* Iterator, but every Subscriber directly attached to the Publisher of this
|
|
|
|
|
* stream will see an individual flow of elements (always starting from the
|
|
|
|
|
* beginning) regardless of when they subscribed.
|
|
|
|
|
*/
|
2015-06-06 16:07:35 +02:00
|
|
|
def apply[T](iterable: immutable.Iterable[T]): Source[T, Unit] =
|
2015-06-06 16:45:01 +02:00
|
|
|
Source.single(iterable).mapConcat(id).withAttributes(DefaultAttributes.iterableSource)
|
2015-01-28 14:19:50 +01:00
|
|
|
|
|
|
|
|
/**
|
|
|
|
|
* Start a new `Source` from the given `Future`. The stream will consist of
|
|
|
|
|
* one element when the `Future` is completed with a successful value, which
|
|
|
|
|
* may happen before or after materializing the `Flow`.
|
2015-03-05 12:21:17 +01:00
|
|
|
* The stream terminates with a failure if the `Future` is completed with a failure.
|
2015-01-28 14:19:50 +01:00
|
|
|
*/
|
|
|
|
|
def apply[T](future: Future[T]): Source[T, Unit] =
|
2015-07-06 22:00:21 +02:00
|
|
|
new Source(
|
|
|
|
|
new PublisherSource(
|
|
|
|
|
SingleElementPublisher(future, "FutureSource"),
|
|
|
|
|
DefaultAttributes.futureSource,
|
|
|
|
|
shape("FutureSource"))).mapAsyncUnordered(1)(id)
|
2014-09-03 21:54:18 +02:00
|
|
|
|
|
|
|
|
/**
|
2015-01-26 14:16:57 +01:00
|
|
|
* Elements are emitted periodically with the specified interval.
|
2014-09-03 21:54:18 +02:00
|
|
|
* The tick element will be delivered to downstream consumers that has requested any elements.
|
|
|
|
|
* If a consumer has not requested any elements at the point in time when the tick
|
|
|
|
|
* element is produced it will not receive that tick element later. It will
|
|
|
|
|
* receive new tick elements as soon as it has requested more elements.
|
|
|
|
|
*/
|
2015-01-28 14:19:50 +01:00
|
|
|
def apply[T](initialDelay: FiniteDuration, interval: FiniteDuration, tick: T): Source[T, Cancellable] =
|
2015-09-18 14:30:43 +02:00
|
|
|
wrap(new TickSource[T](initialDelay, interval, tick))
|
2014-10-02 13:34:27 +02:00
|
|
|
|
2014-10-27 09:48:54 +02:00
|
|
|
/**
|
|
|
|
|
* Create a `Source` with one element.
|
|
|
|
|
* Every connected `Sink` of this stream will see an individual stream consisting of one element.
|
|
|
|
|
*/
|
2015-03-05 12:21:17 +01:00
|
|
|
def single[T](element: T): Source[T, Unit] =
|
2015-07-06 22:00:21 +02:00
|
|
|
new Source(
|
|
|
|
|
new PublisherSource(
|
|
|
|
|
SingleElementPublisher(element, "SingleSource"),
|
|
|
|
|
DefaultAttributes.singleSource,
|
|
|
|
|
shape("SingleSource")))
|
2014-10-27 09:48:54 +02:00
|
|
|
|
2015-02-26 12:36:46 +01:00
|
|
|
/**
|
|
|
|
|
* Create a `Source` that will continually emit the given element.
|
|
|
|
|
*/
|
2015-10-16 01:55:20 +02:00
|
|
|
def repeat[T](element: T): Source[T, Unit] = {
|
|
|
|
|
ReactiveStreamsCompliance.requireNonNullElement(element)
|
2015-07-06 22:00:21 +02:00
|
|
|
new Source(
|
|
|
|
|
new PublisherSource(
|
|
|
|
|
SingleElementPublisher(
|
|
|
|
|
new immutable.Iterable[T] {
|
|
|
|
|
override val iterator: Iterator[T] = Iterator.continually(element)
|
2015-10-16 01:55:20 +02:00
|
|
|
|
2015-07-06 22:00:21 +02:00
|
|
|
override def toString: String = "repeat(" + element + ")"
|
|
|
|
|
}, "RepeatSource"),
|
|
|
|
|
DefaultAttributes.repeat,
|
|
|
|
|
shape("RepeatSource"))).mapConcat(id)
|
2015-10-16 01:55:20 +02:00
|
|
|
}
|
2015-02-26 12:36:46 +01:00
|
|
|
|
2014-10-27 09:48:54 +02:00
|
|
|
/**
|
2014-10-30 17:04:36 +01:00
|
|
|
* A `Source` with no elements, i.e. an empty stream that is completed immediately for every connected `Sink`.
|
2014-10-27 09:48:54 +02:00
|
|
|
*/
|
2015-03-05 12:21:17 +01:00
|
|
|
def empty[T]: Source[T, Unit] = _empty
|
2015-07-06 22:00:21 +02:00
|
|
|
private[this] val _empty: Source[Nothing, Unit] =
|
|
|
|
|
new Source(
|
|
|
|
|
new PublisherSource[Nothing](
|
|
|
|
|
EmptyPublisher,
|
|
|
|
|
DefaultAttributes.emptySource,
|
|
|
|
|
shape("EmptySource")))
|
2014-10-27 09:48:54 +02:00
|
|
|
|
2015-01-29 10:21:54 +01:00
|
|
|
/**
|
|
|
|
|
* Create a `Source` with no elements, which does not complete its downstream,
|
|
|
|
|
* until externally triggered to do so.
|
|
|
|
|
*
|
|
|
|
|
* It materializes a [[scala.concurrent.Promise]] which will be completed
|
|
|
|
|
* when the downstream stage of this source cancels. This promise can also
|
2015-09-28 22:23:59 -07:00
|
|
|
* be used to externally trigger completion, which the source then signals
|
2015-01-29 10:21:54 +01:00
|
|
|
* to its downstream.
|
|
|
|
|
*/
|
2015-03-05 12:21:17 +01:00
|
|
|
def lazyEmpty[T]: Source[T, Promise[Unit]] =
|
2015-04-20 21:04:03 +02:00
|
|
|
new Source(new LazyEmptySource[T](DefaultAttributes.lazyEmptySource, shape("LazyEmptySource")))
|
2015-01-28 14:19:50 +01:00
|
|
|
|
|
|
|
|
/**
|
|
|
|
|
* Create a `Source` that immediately ends the stream with the `cause` error to every connected `Sink`.
|
|
|
|
|
*/
|
2015-04-20 21:04:03 +02:00
|
|
|
def failed[T](cause: Throwable): Source[T, Unit] =
|
2015-07-06 22:00:21 +02:00
|
|
|
new Source(
|
|
|
|
|
new PublisherSource(
|
|
|
|
|
ErrorPublisher(cause, "FailedSource")[T],
|
|
|
|
|
DefaultAttributes.failedSource,
|
|
|
|
|
shape("FailedSource")))
|
2014-10-27 09:48:54 +02:00
|
|
|
|
2014-10-17 14:05:50 +02:00
|
|
|
/**
|
|
|
|
|
* Creates a `Source` that is materialized as a [[org.reactivestreams.Subscriber]]
|
|
|
|
|
*/
|
2015-03-05 12:21:17 +01:00
|
|
|
def subscriber[T]: Source[T, Subscriber[T]] =
|
2015-04-20 21:04:03 +02:00
|
|
|
new Source(new SubscriberSource[T](DefaultAttributes.subscriberSource, shape("SubscriberSource")))
|
2015-01-28 14:19:50 +01:00
|
|
|
|
2015-03-31 15:13:57 +02:00
|
|
|
/**
|
|
|
|
|
* Creates a `Source` that is materialized to an [[akka.actor.ActorRef]] which points to an Actor
|
2015-10-09 15:11:01 -04:00
|
|
|
* created according to the passed in [[akka.actor.Props]]. Actor created by the `props` must
|
2015-03-31 15:13:57 +02:00
|
|
|
* be [[akka.stream.actor.ActorPublisher]].
|
|
|
|
|
*/
|
2015-10-09 15:11:01 -04:00
|
|
|
def actorPublisher[T](props: Props): Source[T, ActorRef] = {
|
|
|
|
|
require(classOf[ActorPublisher[_]].isAssignableFrom(props.actorClass()), "Actor must be ActorPublisher")
|
2015-04-20 21:04:03 +02:00
|
|
|
new Source(new ActorPublisherSource(props, DefaultAttributes.actorPublisherSource, shape("ActorPublisherSource")))
|
2015-10-09 15:11:01 -04:00
|
|
|
}
|
2015-03-31 15:13:57 +02:00
|
|
|
|
|
|
|
|
/**
|
|
|
|
|
* Creates a `Source` that is materialized as an [[akka.actor.ActorRef]].
|
|
|
|
|
* Messages sent to this actor will be emitted to the stream if there is demand from downstream,
|
|
|
|
|
* otherwise they will be buffered until request for demand is received.
|
|
|
|
|
*
|
|
|
|
|
* Depending on the defined [[akka.stream.OverflowStrategy]] it might drop elements if
|
|
|
|
|
* there is no space available in the buffer.
|
|
|
|
|
*
|
2015-07-09 10:18:18 +02:00
|
|
|
* The strategy [[akka.stream.OverflowStrategy.backpressure]] is not supported, and an
|
|
|
|
|
* IllegalArgument("Backpressure overflowStrategy not supported") will be thrown if it is passed as argument.
|
|
|
|
|
*
|
2015-03-31 15:13:57 +02:00
|
|
|
* The buffer can be disabled by using `bufferSize` of 0 and then received messages are dropped
|
|
|
|
|
* if there is no demand from downstream. When `bufferSize` is 0 the `overflowStrategy` does
|
|
|
|
|
* not matter.
|
|
|
|
|
*
|
2015-06-19 10:03:55 +03:00
|
|
|
* The stream can be completed successfully by sending the actor reference an [[akka.actor.Status.Success]]
|
2015-09-28 22:23:59 -07:00
|
|
|
* message in which case already buffered elements will be signaled before signaling completion,
|
|
|
|
|
* or by sending a [[akka.actor.PoisonPill]] in which case completion will be signaled immediately.
|
2015-03-31 15:13:57 +02:00
|
|
|
*
|
|
|
|
|
* The stream can be completed with failure by sending [[akka.actor.Status.Failure]] to the
|
2015-04-28 09:48:46 +02:00
|
|
|
* actor reference. In case the Actor is still draining its internal buffer (after having received
|
2015-09-28 22:23:59 -07:00
|
|
|
* an [[akka.actor.Status.Success]]) before signaling completion and it receives a [[akka.actor.Status.Failure]],
|
|
|
|
|
* the failure will be signaled downstream immediately (instead of the completion signal).
|
2015-03-31 15:13:57 +02:00
|
|
|
*
|
2015-09-28 22:23:59 -07:00
|
|
|
* The actor will be stopped when the stream is completed, failed or canceled from downstream,
|
2015-03-31 15:13:57 +02:00
|
|
|
* i.e. you can watch it to get notified when that happens.
|
|
|
|
|
*
|
|
|
|
|
* @param bufferSize The size of the buffer in element count
|
|
|
|
|
* @param overflowStrategy Strategy that is used when incoming elements cannot fit inside the buffer
|
|
|
|
|
*/
|
|
|
|
|
def actorRef[T](bufferSize: Int, overflowStrategy: OverflowStrategy): Source[T, ActorRef] = {
|
|
|
|
|
require(bufferSize >= 0, "bufferSize must be greater than or equal to 0")
|
|
|
|
|
require(overflowStrategy != OverflowStrategy.Backpressure, "Backpressure overflowStrategy not supported")
|
2015-04-20 21:04:03 +02:00
|
|
|
new Source(new ActorRefSource(bufferSize, overflowStrategy, DefaultAttributes.actorRefSource, shape("ActorRefSource")))
|
2015-03-31 15:13:57 +02:00
|
|
|
}
|
|
|
|
|
|
2015-06-29 23:47:31 -04:00
|
|
|
/**
|
|
|
|
|
* Combines several sources with fun-in strategy like `Merge` or `Concat` and returns `Source`.
|
|
|
|
|
*/
|
|
|
|
|
def combine[T, U](first: Source[T, _], second: Source[T, _], rest: Source[T, _]*)(strategy: Int ⇒ Graph[UniformFanInShape[T, U], Unit]): Source[U, Unit] =
|
|
|
|
|
Source.wrap(FlowGraph.partial() { implicit b ⇒
|
|
|
|
|
import FlowGraph.Implicits._
|
|
|
|
|
val c = b.add(strategy(rest.size + 2))
|
|
|
|
|
first ~> c.in(0)
|
|
|
|
|
second ~> c.in(1)
|
|
|
|
|
|
|
|
|
|
@tailrec def combineRest(idx: Int, i: Iterator[Source[T, _]]): SourceShape[U] =
|
|
|
|
|
if (i.hasNext) {
|
|
|
|
|
i.next() ~> c.in(idx)
|
|
|
|
|
combineRest(idx + 1, i)
|
|
|
|
|
} else SourceShape(c.out)
|
|
|
|
|
|
|
|
|
|
combineRest(2, rest.iterator)
|
|
|
|
|
})
|
|
|
|
|
|
2015-08-19 23:04:20 -04:00
|
|
|
/**
|
|
|
|
|
* Creates a `Source` that is materialized as an [[akka.stream.SourceQueue]].
|
|
|
|
|
* You can push elements to the queue and they will be emitted to the stream if there is demand from downstream,
|
|
|
|
|
* otherwise they will be buffered until request for demand is received.
|
|
|
|
|
*
|
|
|
|
|
* Depending on the defined [[akka.stream.OverflowStrategy]] it might drop elements if
|
|
|
|
|
* there is no space available in the buffer.
|
|
|
|
|
*
|
|
|
|
|
* Acknowledgement mechanism is available.
|
|
|
|
|
* [[akka.stream.SourceQueue.offer]] returns ``Future[Boolean]`` which completes with true
|
|
|
|
|
* if element was added to buffer or sent downstream. It completes
|
|
|
|
|
* with false if element was dropped.
|
|
|
|
|
*
|
|
|
|
|
* The strategy [[akka.stream.OverflowStrategy.backpressure]] will not complete `offer():Future` until buffer is full.
|
|
|
|
|
*
|
|
|
|
|
* The buffer can be disabled by using `bufferSize` of 0 and then received messages are dropped
|
|
|
|
|
* if there is no demand from downstream. When `bufferSize` is 0 the `overflowStrategy` does
|
|
|
|
|
* not matter.
|
|
|
|
|
*
|
|
|
|
|
* @param bufferSize The size of the buffer in element count
|
|
|
|
|
* @param overflowStrategy Strategy that is used when incoming elements cannot fit inside the buffer
|
|
|
|
|
* @param timeout Timeout for ``SourceQueue.offer(T):Future[Boolean]``
|
|
|
|
|
*/
|
|
|
|
|
def queue[T](bufferSize: Int, overflowStrategy: OverflowStrategy, timeout: FiniteDuration = 5.seconds): Source[T, SourceQueue[T]] = {
|
|
|
|
|
require(bufferSize >= 0, "bufferSize must be greater than or equal to 0")
|
|
|
|
|
new Source(new AcknowledgeSource(bufferSize, overflowStrategy, DefaultAttributes.acknowledgeSource, shape("AcknowledgeSource")))
|
|
|
|
|
}
|
|
|
|
|
|
2015-01-28 14:19:50 +01:00
|
|
|
}
|