object Source
- Source
- Source.scala
- Alphabetic
- By Inheritance
- Source
- AnyRef
- Any
- Hide All
- Show All
- Public
- All
Value Members
- 
      
      
      
        
      
    
      
        final 
        def
      
      
        !=(arg0: Any): Boolean
      
      
      - Definition Classes
- AnyRef → Any
 
- 
      
      
      
        
      
    
      
        final 
        def
      
      
        ##(): Int
      
      
      - Definition Classes
- AnyRef → Any
 
- 
      
      
      
        
      
    
      
        final 
        def
      
      
        ==(arg0: Any): Boolean
      
      
      - Definition Classes
- AnyRef → Any
 
- 
      
      
      
        
      
    
      
        
        def
      
      
        actorPublisher[T](props: Props): Source[T, ActorRef]
      
      
      Creates a Sourcethat is materialized to an akka.actor.ActorRef which points to an Actor created according to the passed in akka.actor.Props.Creates a Sourcethat is materialized to an akka.actor.ActorRef which points to an Actor created according to the passed in akka.actor.Props. Actor created by thepropsmust be akka.stream.actor.ActorPublisher.
- 
      
      
      
        
      
    
      
        
        def
      
      
        actorRef[T](bufferSize: Int, overflowStrategy: OverflowStrategy): Source[T, ActorRef]
      
      
      Creates a Sourcethat is materialized as an akka.actor.ActorRef.Creates a Sourcethat 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. 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. The buffer can be disabled by using bufferSizeof 0 and then received messages are dropped if there is no demand from downstream. WhenbufferSizeis 0 theoverflowStrategydoes not matter. An async boundary is added after this Source; as such, it is never safe to assume the downstream will always generate demand.The stream can be completed successfully by sending the actor reference a akka.actor.Status.Success (whose content will be ignored) in which case already buffered elements will be signaled before signaling completion, or by sending akka.actor.PoisonPill in which case completion will be signaled immediately. The stream can be completed with failure by sending a akka.actor.Status.Failure to the actor reference. In case the Actor is still draining its internal buffer (after having received a 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). The actor will be stopped when the stream is completed, failed or canceled from downstream, i.e. you can watch it to get notified when that happens. See also akka.stream.scaladsl.Source.queue. - bufferSize
- The size of the buffer in element count 
- overflowStrategy
- Strategy that is used when incoming elements cannot fit inside the buffer 
 
- 
      
      
      
        
      
    
      
        
        def
      
      
        apply[T](iterable: Iterable[T]): Source[T, NotUsed]
      
      
      Helper to create Source from Iterable.Helper to create Source from Iterable. Example usage:Source(Seq(1,2,3))Starts a new Sourcefrom the givenIterable. This is like starting from an 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.
- 
      
      
      
        
      
    
      
        final 
        def
      
      
        asInstanceOf[T0]: T0
      
      
      - Definition Classes
- Any
 
- 
      
      
      
        
      
    
      
        
        def
      
      
        asSubscriber[T]: Source[T, Subscriber[T]]
      
      
      Creates a Sourcethat is materialized as a org.reactivestreams.Subscriber
- 
      
      
      
        
      
    
      
        
        def
      
      
        clone(): AnyRef
      
      
      - Attributes
- protected[java.lang]
- Definition Classes
- AnyRef
- Annotations
- @throws( ... )
 
- 
      
      
      
        
      
    
      
        
        def
      
      
        combine[T, U](first: Source[T, _], second: Source[T, _], rest: Source[T, _]*)(strategy: (Int) ⇒ Graph[UniformFanInShape[T, U], NotUsed]): Source[U, NotUsed]
      
      
      Combines several sources with fun-in strategy like MergeorConcatand returnsSource.
- 
      
      
      
        
      
    
      
        
        def
      
      
        cycle[T](f: () ⇒ Iterator[T]): Source[T, NotUsed]
      
      
      Create Source that will continually produce given elements in specified order. Create Source that will continually produce given elements in specified order. Start a new 'cycled' Sourcefrom the given elements. The producer stream of elements will continue infinitely by repeating the sequence of elements provided by function parameter.
- 
      
      
      
        
      
    
      
        
        def
      
      
        empty[T]: Source[T, NotUsed]
      
      
      A Sourcewith no elements, i.e.A Sourcewith no elements, i.e. an empty stream that is completed immediately for every connectedSink.
- 
      
      
      
        
      
    
      
        final 
        def
      
      
        eq(arg0: AnyRef): Boolean
      
      
      - Definition Classes
- AnyRef
 
- 
      
      
      
        
      
    
      
        
        def
      
      
        equals(arg0: Any): Boolean
      
      
      - Definition Classes
- AnyRef → Any
 
- 
      
      
      
        
      
    
      
        
        def
      
      
        failed[T](cause: Throwable): Source[T, NotUsed]
      
      
      Create a Sourcethat immediately ends the stream with thecauseerror to every connectedSink.
- 
      
      
      
        
      
    
      
        
        def
      
      
        finalize(): Unit
      
      
      - Attributes
- protected[java.lang]
- Definition Classes
- AnyRef
- Annotations
- @throws( classOf[java.lang.Throwable] )
 
- 
      
      
      
        
      
    
      
        
        def
      
      
        fromCompletionStage[T](future: CompletionStage[T]): Source[T, NotUsed]
      
      
      Start a new Sourcefrom the givenFuture.Start a new Sourcefrom the givenFuture. The stream will consist of one element when theFutureis completed with a successful value, which may happen before or after materializing theFlow. The stream terminates with a failure if theFutureis completed with a failure.
- 
      
      
      
        
      
    
      
        
        def
      
      
        fromFuture[T](future: Future[T]): Source[T, NotUsed]
      
      
      Start a new Sourcefrom the givenFuture.Start a new Sourcefrom the givenFuture. The stream will consist of one element when theFutureis completed with a successful value, which may happen before or after materializing theFlow. The stream terminates with a failure if theFutureis completed with a failure.
- 
      
      
      
        
      
    
      
        
        def
      
      
        fromGraph[T, M](g: Graph[SourceShape[T], M]): Source[T, M]
      
      
      A graph with the shape of a source logically is a source, this method makes it so also in type. 
- 
      
      
      
        
      
    
      
        
        def
      
      
        fromIterator[T](f: () ⇒ Iterator[T]): Source[T, NotUsed]
      
      
      Helper to create Source from Iterator.Helper to create Source from Iterator. Example usage:Source.fromIterator(() => Iterator.from(0))Start a new Sourcefrom the given function that produces anIterator. The produced stream of elements will continue until the iterator runs empty or fails during evaluation of thenext()method. Elements are pulled out of the iterator in accordance with the demand coming from the downstream transformation steps.
- 
      
      
      
        
      
    
      
        
        def
      
      
        fromPublisher[T](publisher: Publisher[T]): Source[T, NotUsed]
      
      
      Helper to create Source from Publisher.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. 
- 
      
      
      
        
      
    
      
        final 
        def
      
      
        getClass(): Class[_]
      
      
      - Definition Classes
- AnyRef → Any
 
- 
      
      
      
        
      
    
      
        
        def
      
      
        hashCode(): Int
      
      
      - Definition Classes
- AnyRef → Any
 
- 
      
      
      
        
      
    
      
        final 
        def
      
      
        isInstanceOf[T0]: Boolean
      
      
      - Definition Classes
- Any
 
- 
      
      
      
        
      
    
      
        
        def
      
      
        lazily[T, M](create: () ⇒ Source[T, M]): Source[T, Future[M]]
      
      
      Creates a Sourcethat is not materialized until there is downstream demand, when the source gets materialized the materialized future is completed with its value, if downstream cancels or fails without any demand the create factory is never called and the materializedFutureis failed.
- 
      
      
      
        
      
    
      
        
        def
      
      
        maybe[T]: Source[T, Promise[Option[T]]]
      
      
      Create a Sourcewhich materializes a scala.concurrent.Promise which controls what element will be emitted by the Source.Create a Sourcewhich materializes a scala.concurrent.Promise which controls what element will be emitted by the Source. If the materialized promise is completed with a Some, that value will be produced downstream, followed by completion. If the materialized promise is completed with a None, no value will be produced downstream and completion will be signalled immediately. If the materialized promise is completed with a failure, then the returned source will terminate with that error. If the downstream of this source cancels before the promise has been completed, then the promise will be completed with None.
- 
      
      
      
        
      
    
      
        final 
        def
      
      
        ne(arg0: AnyRef): Boolean
      
      
      - Definition Classes
- AnyRef
 
- 
      
      
      
        
      
    
      
        final 
        def
      
      
        notify(): Unit
      
      
      - Definition Classes
- AnyRef
 
- 
      
      
      
        
      
    
      
        final 
        def
      
      
        notifyAll(): Unit
      
      
      - Definition Classes
- AnyRef
 
- 
      
      
      
        
      
    
      
        
        def
      
      
        queue[T](bufferSize: Int, overflowStrategy: OverflowStrategy): Source[T, SourceQueueWithComplete[T]]
      
      
      Creates a Sourcethat is materialized as an akka.stream.scaladsl.SourceQueue.Creates a Sourcethat is materialized as an akka.stream.scaladsl.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. Elements in the buffer will be discarded if downstream is terminated.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.scaladsl.SourceQueue.offer returns Future[QueueOfferResult]which completes withQueueOfferResult.Enqueuedif element was added to buffer or sent downstream. It completes withQueueOfferResult.Droppedif element was dropped. Can also complete withQueueOfferResult.Failure- when stream failed orQueueOfferResult.QueueClosedwhen downstream is completed.The strategy akka.stream.OverflowStrategy.backpressure will not complete last offer():Futurecall when buffer is full.You can watch accessibility of stream with akka.stream.scaladsl.SourceQueue.watchCompletion. It returns future that completes with success when stream is completed or fail when stream is failed. The buffer can be disabled by using bufferSizeof 0 and then received message will wait for downstream demand unless there is another message waiting for downstream demand, in that case offer result will be completed according to the overflow strategy.SourceQueue that current source is materialized to is for single thread usage only. - bufferSize
- size of buffer in element count 
- overflowStrategy
- Strategy that is used when incoming elements cannot fit inside the buffer 
 
- 
      
      
      
        
      
    
      
        
        def
      
      
        repeat[T](element: T): Source[T, NotUsed]
      
      
      Create a Sourcethat will continually emit the given element.
- 
      
      
      
        
      
    
      
        
        def
      
      
        shape[T](name: String): SourceShape[T]
      
      
      INTERNAL API 
- 
      
      
      
        
      
    
      
        
        def
      
      
        single[T](element: T): Source[T, NotUsed]
      
      
      Create a Sourcewith one element.Create a Sourcewith one element. Every connectedSinkof this stream will see an individual stream consisting of one element.
- 
      
      
      
        
      
    
      
        final 
        def
      
      
        synchronized[T0](arg0: ⇒ T0): T0
      
      
      - Definition Classes
- AnyRef
 
- 
      
      
      
        
      
    
      
        
        def
      
      
        tick[T](initialDelay: FiniteDuration, interval: FiniteDuration, tick: T): Source[T, Cancellable]
      
      
      Elements are emitted periodically with the specified interval. Elements are emitted periodically with the specified interval. 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. 
- 
      
      
      
        
      
    
      
        
        def
      
      
        toString(): String
      
      
      - Definition Classes
- AnyRef → Any
 
- 
      
      
      
        
      
    
      
        
        def
      
      
        unfold[S, E](s: S)(f: (S) ⇒ Option[(S, E)]): Source[E, NotUsed]
      
      
      Create a Sourcethat will unfold a value of typeSinto a pair of the next stateSand output elements of typeE.Create a Sourcethat will unfold a value of typeSinto a pair of the next stateSand output elements of typeE.For example, all the Fibonacci numbers under 10M: Source.unfold(0 → 1) { case (a, _) if a > 10000000 ⇒ None case (a, b) ⇒ Some((b → (a + b)) → a) } 
- 
      
      
      
        
      
    
      
        
        def
      
      
        unfoldAsync[S, E](s: S)(f: (S) ⇒ Future[Option[(S, E)]]): Source[E, NotUsed]
      
      
      Same as unfold, but uses an async function to generate the next state-element tuple. Same as unfold, but uses an async function to generate the next state-element tuple. async fibonacci example: Source.unfoldAsync(0 → 1) { case (a, _) if a > 10000000 ⇒ Future.successful(None) case (a, b) ⇒ Future{ Thread.sleep(1000) Some((b → (a + b)) → a) } } 
- 
      
      
      
        
      
    
      
        
        def
      
      
        unfoldResource[T, S](create: () ⇒ S, read: (S) ⇒ Option[T], close: (S) ⇒ Unit): Source[T, NotUsed]
      
      
      Start a new Sourcefrom some resource which can be opened, read and closed.Start a new Sourcefrom some resource which can be opened, read and closed. Interaction with resource happens in a blocking way.Example: Source.unfoldResource( () => new BufferedReader(new FileReader("...")), reader => Option(reader.readLine()), reader => reader.close()) You can use the supervision strategy to handle exceptions for readfunction. All exceptions thrown bycreateorclosewill fail the stream.Restartsupervision strategy will close and create blocking IO again. Default strategy isStopwhich means that stream will be terminated on error inreadfunction by default.You can configure the default dispatcher for this Source by changing the akka.stream.blocking-io-dispatcheror set it for a given Source by using ActorAttributes.- create
- - function that is called on stream start and creates/opens resource. 
- read
- - function that reads data from opened resource. It is called each time backpressure signal is received. Stream calls close and completes when - readreturns None.
- close
- - function that closes resource 
 
- 
      
      
      
        
      
    
      
        
        def
      
      
        unfoldResourceAsync[T, S](create: () ⇒ Future[S], read: (S) ⇒ Future[Option[T]], close: (S) ⇒ Future[Done]): Source[T, NotUsed]
      
      
      Start a new Sourcefrom some resource which can be opened, read and closed.Start a new Sourcefrom some resource which can be opened, read and closed. It's similar tounfoldResourcebut takes functions that returnFuturesinstead of plain values.You can use the supervision strategy to handle exceptions for readfunction or failures of producedFutures. All exceptions thrown bycreateorcloseas well as fails of returned futures will fail the stream.Restartsupervision strategy will close and create resource. Default strategy isStopwhich means that stream will be terminated on error inreadfunction (or future) by default.You can configure the default dispatcher for this Source by changing the akka.stream.blocking-io-dispatcheror set it for a given Source by using ActorAttributes.- create
- - function that is called on stream start and creates/opens resource. 
- read
- - function that reads data from opened resource. It is called each time backpressure signal is received. Stream calls close and completes when - Futurefrom read function returns None.
- close
- - function that closes resource 
 
- 
      
      
      
        
      
    
      
        final 
        def
      
      
        wait(): Unit
      
      
      - Definition Classes
- AnyRef
- Annotations
- @throws( ... )
 
- 
      
      
      
        
      
    
      
        final 
        def
      
      
        wait(arg0: Long, arg1: Int): Unit
      
      
      - Definition Classes
- AnyRef
- Annotations
- @throws( ... )
 
- 
      
      
      
        
      
    
      
        final 
        def
      
      
        wait(arg0: Long): Unit
      
      
      - Definition Classes
- AnyRef
- Annotations
- @throws( ... )
 
- 
      
      
      
        
      
    
      
        
        def
      
      
        zipN[T](sources: Seq[Source[T, _]]): Source[Seq[T], NotUsed]
      
      
      Combine the elements of multiple streams into a stream of sequences. 
-  def zipWithN[T, O](zipper: (Seq[T]) ⇒ O)(sources: Seq[Source[T, _]]): Source[O, NotUsed]