parent
48e4f11dfd
commit
3ae85e8cd0
8 changed files with 27 additions and 27 deletions
|
|
@ -53,7 +53,7 @@ class SendQueueBenchmark {
|
||||||
val N = 100000
|
val N = 100000
|
||||||
val burstSize = 1000
|
val burstSize = 1000
|
||||||
|
|
||||||
val source = Source.queue[Int](1024, OverflowStrategy.dropBuffer)
|
val source = Source.queue[Int](1024)
|
||||||
|
|
||||||
val (queue, killSwitch) = source
|
val (queue, killSwitch) = source
|
||||||
.viaMat(KillSwitches.single)(Keep.both)
|
.viaMat(KillSwitches.single)(Keep.both)
|
||||||
|
|
|
||||||
|
|
@ -749,8 +749,8 @@ public class IntegrationDocTest extends AbstractJavaTest {
|
||||||
int bufferSize = 10;
|
int bufferSize = 10;
|
||||||
int elementsToProcess = 5;
|
int elementsToProcess = 5;
|
||||||
|
|
||||||
SourceQueueWithComplete<Integer> sourceQueue =
|
BoundedSourceQueue<Integer> sourceQueue =
|
||||||
Source.<Integer>queue(bufferSize, OverflowStrategy.backpressure())
|
Source.<Integer>queue(bufferSize)
|
||||||
.throttle(elementsToProcess, Duration.ofSeconds(3))
|
.throttle(elementsToProcess, Duration.ofSeconds(3))
|
||||||
.map(x -> x * x)
|
.map(x -> x * x)
|
||||||
.to(Sink.foreach(x -> System.out.println("got: " + x)))
|
.to(Sink.foreach(x -> System.out.println("got: " + x)))
|
||||||
|
|
|
||||||
|
|
@ -22,6 +22,7 @@ import scala.concurrent.ExecutionContext
|
||||||
import java.util.concurrent.atomic.AtomicInteger
|
import java.util.concurrent.atomic.AtomicInteger
|
||||||
|
|
||||||
import akka.stream.scaladsl.Flow
|
import akka.stream.scaladsl.Flow
|
||||||
|
import org.scalacheck.Gen.const
|
||||||
|
|
||||||
object IntegrationDocSpec {
|
object IntegrationDocSpec {
|
||||||
import TwitterStreamQuickstartDocSpec._
|
import TwitterStreamQuickstartDocSpec._
|
||||||
|
|
@ -469,7 +470,7 @@ class IntegrationDocSpec extends AkkaSpec(IntegrationDocSpec.config) {
|
||||||
val elementsToProcess = 5
|
val elementsToProcess = 5
|
||||||
|
|
||||||
val queue = Source
|
val queue = Source
|
||||||
.queue[Int](bufferSize, OverflowStrategy.backpressure)
|
.queue[Int](bufferSize)
|
||||||
.throttle(elementsToProcess, 3.second)
|
.throttle(elementsToProcess, 3.second)
|
||||||
.map(x => x * x)
|
.map(x => x * x)
|
||||||
.toMat(Sink.foreach(x => println(s"completed $x")))(Keep.left)
|
.toMat(Sink.foreach(x => println(s"completed $x")))(Keep.left)
|
||||||
|
|
@ -479,7 +480,7 @@ class IntegrationDocSpec extends AkkaSpec(IntegrationDocSpec.config) {
|
||||||
|
|
||||||
implicit val ec = system.dispatcher
|
implicit val ec = system.dispatcher
|
||||||
source
|
source
|
||||||
.mapAsync(1)(x => {
|
.map(x => {
|
||||||
queue.offer(x).map {
|
queue.offer(x).map {
|
||||||
case QueueOfferResult.Enqueued => println(s"enqueued $x")
|
case QueueOfferResult.Enqueued => println(s"enqueued $x")
|
||||||
case QueueOfferResult.Dropped => println(s"dropped $x")
|
case QueueOfferResult.Dropped => println(s"dropped $x")
|
||||||
|
|
|
||||||
|
|
@ -636,9 +636,9 @@ public class SourceTest extends StreamTest {
|
||||||
|
|
||||||
@Test
|
@Test
|
||||||
public void mustBeAbleToUseQueue() throws Exception {
|
public void mustBeAbleToUseQueue() throws Exception {
|
||||||
final Pair<SourceQueueWithComplete<String>, CompletionStage<List<String>>> x =
|
final Pair<BoundedSourceQueue<String>, CompletionStage<List<String>>> x =
|
||||||
Flow.of(String.class).runWith(Source.queue(2, OverflowStrategy.fail()), Sink.seq(), system);
|
Flow.of(String.class).runWith(Source.queue(2), Sink.seq(), system);
|
||||||
final SourceQueueWithComplete<String> source = x.first();
|
final BoundedSourceQueue<String> source = x.first();
|
||||||
final CompletionStage<List<String>> result = x.second();
|
final CompletionStage<List<String>> result = x.second();
|
||||||
source.offer("hello");
|
source.offer("hello");
|
||||||
source.offer("world");
|
source.offer("world");
|
||||||
|
|
@ -833,20 +833,19 @@ public class SourceTest extends StreamTest {
|
||||||
@Test
|
@Test
|
||||||
public void mustBeAbleToCombineMat() throws Exception {
|
public void mustBeAbleToCombineMat() throws Exception {
|
||||||
final TestKit probe = new TestKit(system);
|
final TestKit probe = new TestKit(system);
|
||||||
final Source<Integer, SourceQueueWithComplete<Integer>> source1 =
|
final Source<Integer, BoundedSourceQueue<Integer>> source1 = Source.queue(2);
|
||||||
Source.queue(1, OverflowStrategy.dropNew());
|
|
||||||
final Source<Integer, NotUsed> source2 = Source.from(Arrays.asList(2, 3));
|
final Source<Integer, NotUsed> source2 = Source.from(Arrays.asList(2, 3));
|
||||||
|
|
||||||
// compiler to check the correct materialized value of type = SourceQueueWithComplete<Integer>
|
// compiler to check the correct materialized value of type = BoundedSourceQueue<Integer>
|
||||||
// available
|
// available
|
||||||
final Source<Integer, SourceQueueWithComplete<Integer>> combined =
|
final Source<Integer, BoundedSourceQueue<Integer>> combined =
|
||||||
Source.combineMat(
|
Source.combineMat(
|
||||||
source1,
|
source1,
|
||||||
source2,
|
source2,
|
||||||
width -> Concat.create(width),
|
width -> Concat.create(width),
|
||||||
Keep.left()); // Keep.left() (i.e. preserve queueSource's materialized value)
|
Keep.left()); // Keep.left() (i.e. preserve queueSource's materialized value)
|
||||||
|
|
||||||
SourceQueueWithComplete<Integer> queue =
|
BoundedSourceQueue<Integer> queue =
|
||||||
combined
|
combined
|
||||||
.toMat(
|
.toMat(
|
||||||
Sink.foreach(elem -> probe.getRef().tell(elem, ActorRef.noSender())), Keep.left())
|
Sink.foreach(elem -> probe.getRef().tell(elem, ActorRef.noSender())), Keep.left())
|
||||||
|
|
|
||||||
|
|
@ -656,7 +656,7 @@ class FlowGroupBySpec extends StreamSpec("""
|
||||||
"not block all substreams when one is blocked but has a buffer in front" in assertAllStagesStopped {
|
"not block all substreams when one is blocked but has a buffer in front" in assertAllStagesStopped {
|
||||||
case class Elem(id: Int, substream: Int, f: () => Any)
|
case class Elem(id: Int, substream: Int, f: () => Any)
|
||||||
val queue = Source
|
val queue = Source
|
||||||
.queue[Elem](3, OverflowStrategy.backpressure)
|
.queue[Elem](3)
|
||||||
.groupBy(2, _.substream)
|
.groupBy(2, _.substream)
|
||||||
.buffer(2, OverflowStrategy.backpressure)
|
.buffer(2, OverflowStrategy.backpressure)
|
||||||
.map { _.f() }
|
.map { _.f() }
|
||||||
|
|
|
||||||
|
|
@ -171,11 +171,11 @@ class SourceSpec extends StreamSpec with DefaultTimeout {
|
||||||
}
|
}
|
||||||
|
|
||||||
"combine from two inputs with combinedMat and take a materialized value" in {
|
"combine from two inputs with combinedMat and take a materialized value" in {
|
||||||
val queueSource = Source.queue[Int](1, OverflowStrategy.dropBuffer)
|
val queueSource = Source.queue[Int](3)
|
||||||
val intSeqSource = Source(1 to 3)
|
val intSeqSource = Source(1 to 3)
|
||||||
|
|
||||||
// compiler to check the correct materialized value of type = SourceQueueWithComplete[Int] available
|
// compiler to check the correct materialized value of type = SourceQueueWithComplete[Int] available
|
||||||
val combined1: Source[Int, SourceQueueWithComplete[Int]] =
|
val combined1: Source[Int, BoundedSourceQueue[Int]] =
|
||||||
Source.combineMat(queueSource, intSeqSource)(Concat(_))(Keep.left) //Keep.left (i.e. preserve queueSource's materialized value)
|
Source.combineMat(queueSource, intSeqSource)(Concat(_))(Keep.left) //Keep.left (i.e. preserve queueSource's materialized value)
|
||||||
|
|
||||||
val (queue1, sinkProbe1) = combined1.toMat(TestSink.probe[Int])(Keep.both).run()
|
val (queue1, sinkProbe1) = combined1.toMat(TestSink.probe[Int])(Keep.both).run()
|
||||||
|
|
@ -192,7 +192,7 @@ class SourceSpec extends StreamSpec with DefaultTimeout {
|
||||||
sinkProbe1.expectNext(3)
|
sinkProbe1.expectNext(3)
|
||||||
|
|
||||||
// compiler to check the correct materialized value of type = SourceQueueWithComplete[Int] available
|
// compiler to check the correct materialized value of type = SourceQueueWithComplete[Int] available
|
||||||
val combined2: Source[Int, SourceQueueWithComplete[Int]] =
|
val combined2: Source[Int, BoundedSourceQueue[Int]] =
|
||||||
//queueSource to be the second of combined source
|
//queueSource to be the second of combined source
|
||||||
Source.combineMat(intSeqSource, queueSource)(Concat(_))(Keep.right) //Keep.right (i.e. preserve queueSource's materialized value)
|
Source.combineMat(intSeqSource, queueSource)(Concat(_))(Keep.right) //Keep.right (i.e. preserve queueSource's materialized value)
|
||||||
|
|
||||||
|
|
@ -390,7 +390,7 @@ class SourceSpec extends StreamSpec with DefaultTimeout {
|
||||||
}
|
}
|
||||||
|
|
||||||
"allow for multiple downstream materialized sources" in {
|
"allow for multiple downstream materialized sources" in {
|
||||||
val matValPoweredSource = Source.queue[String](Int.MaxValue, OverflowStrategy.fail)
|
val matValPoweredSource = Source.queue[String](Int.MaxValue)
|
||||||
val (mat, src) = matValPoweredSource.preMaterialize()
|
val (mat, src) = matValPoweredSource.preMaterialize()
|
||||||
|
|
||||||
val probe1 = src.runWith(TestSink.probe[String])
|
val probe1 = src.runWith(TestSink.probe[String])
|
||||||
|
|
@ -398,25 +398,25 @@ class SourceSpec extends StreamSpec with DefaultTimeout {
|
||||||
|
|
||||||
probe1.request(1)
|
probe1.request(1)
|
||||||
probe2.request(1)
|
probe2.request(1)
|
||||||
mat.offer("One").futureValue
|
mat.offer("One")
|
||||||
probe1.expectNext("One")
|
probe1.expectNext("One")
|
||||||
probe2.expectNext("One")
|
probe2.expectNext("One")
|
||||||
}
|
}
|
||||||
|
|
||||||
"survive cancellations of downstream materialized sources" in {
|
"survive cancellations of downstream materialized sources" in {
|
||||||
val matValPoweredSource = Source.queue[String](Int.MaxValue, OverflowStrategy.fail)
|
val matValPoweredSource = Source.queue[String](Int.MaxValue)
|
||||||
val (mat, src) = matValPoweredSource.preMaterialize()
|
val (mat, src) = matValPoweredSource.preMaterialize()
|
||||||
|
|
||||||
val probe1 = src.runWith(TestSink.probe[String])
|
val probe1 = src.runWith(TestSink.probe[String])
|
||||||
src.runWith(Sink.cancelled)
|
src.runWith(Sink.cancelled)
|
||||||
|
|
||||||
probe1.request(1)
|
probe1.request(1)
|
||||||
mat.offer("One").futureValue
|
mat.offer("One")
|
||||||
probe1.expectNext("One")
|
probe1.expectNext("One")
|
||||||
}
|
}
|
||||||
|
|
||||||
"propagate failures to downstream materialized sources" in {
|
"propagate failures to downstream materialized sources" in {
|
||||||
val matValPoweredSource = Source.queue[String](Int.MaxValue, OverflowStrategy.fail)
|
val matValPoweredSource = Source.queue[String](Int.MaxValue)
|
||||||
val (mat, src) = matValPoweredSource.preMaterialize()
|
val (mat, src) = matValPoweredSource.preMaterialize()
|
||||||
|
|
||||||
val probe1 = src.runWith(TestSink.probe[String])
|
val probe1 = src.runWith(TestSink.probe[String])
|
||||||
|
|
|
||||||
|
|
@ -34,7 +34,7 @@ public class ActorSourceSinkCompileTest {
|
||||||
{
|
{
|
||||||
final ActorRef<String> ref = null;
|
final ActorRef<String> ref = null;
|
||||||
|
|
||||||
Source.<String>queue(10, OverflowStrategy.dropBuffer())
|
Source.<String>queue(10)
|
||||||
.map(s -> s + "!")
|
.map(s -> s + "!")
|
||||||
.to(ActorSink.actorRef(ref, "DONE", ex -> "FAILED: " + ex.getMessage()));
|
.to(ActorSink.actorRef(ref, "DONE", ex -> "FAILED: " + ex.getMessage()));
|
||||||
}
|
}
|
||||||
|
|
@ -42,7 +42,7 @@ public class ActorSourceSinkCompileTest {
|
||||||
{
|
{
|
||||||
final ActorRef<Protocol> ref = null;
|
final ActorRef<Protocol> ref = null;
|
||||||
|
|
||||||
Source.<String>queue(10, OverflowStrategy.dropBuffer())
|
Source.<String>queue(10)
|
||||||
.to(
|
.to(
|
||||||
ActorSink.actorRefWithBackpressure(
|
ActorSink.actorRefWithBackpressure(
|
||||||
ref,
|
ref,
|
||||||
|
|
|
||||||
|
|
@ -33,7 +33,7 @@ class ActorSourceSinkSpec extends ScalaTestWithActorTestKit with AnyWordSpecLike
|
||||||
|
|
||||||
val in =
|
val in =
|
||||||
Source
|
Source
|
||||||
.queue[String](10, OverflowStrategy.dropBuffer)
|
.queue[String](10)
|
||||||
.map(_ + "!")
|
.map(_ + "!")
|
||||||
.to(ActorSink.actorRef(p.ref, "DONE", ex => "FAILED: " + ex.getMessage))
|
.to(ActorSink.actorRef(p.ref, "DONE", ex => "FAILED: " + ex.getMessage))
|
||||||
.run()
|
.run()
|
||||||
|
|
@ -65,7 +65,7 @@ class ActorSourceSinkSpec extends ScalaTestWithActorTestKit with AnyWordSpecLike
|
||||||
|
|
||||||
val in =
|
val in =
|
||||||
Source
|
Source
|
||||||
.queue[String](10, OverflowStrategy.dropBuffer)
|
.queue[String](10)
|
||||||
.to(ActorSink.actorRefWithBackpressure(pilotRef, Msg.apply, Init.apply, "ACK", Complete, _ => Failed))
|
.to(ActorSink.actorRefWithBackpressure(pilotRef, Msg.apply, Init.apply, "ACK", Complete, _ => Failed))
|
||||||
.run()
|
.run()
|
||||||
|
|
||||||
|
|
@ -102,7 +102,7 @@ class ActorSourceSinkSpec extends ScalaTestWithActorTestKit with AnyWordSpecLike
|
||||||
|
|
||||||
val in =
|
val in =
|
||||||
Source
|
Source
|
||||||
.queue[String](10, OverflowStrategy.dropBuffer)
|
.queue[String](10)
|
||||||
.to(ActorSink.actorRefWithBackpressure(pilotRef, Msg.apply, Init.apply, Complete, _ => Failed))
|
.to(ActorSink.actorRefWithBackpressure(pilotRef, Msg.apply, Init.apply, Complete, _ => Failed))
|
||||||
.run()
|
.run()
|
||||||
|
|
||||||
|
|
|
||||||
Loading…
Add table
Add a link
Reference in a new issue