Enable fatal errors for docs (#29570)

This commit is contained in:
Arnout Engelen 2020-09-08 15:10:21 +02:00 committed by GitHub
parent e0ceb71ccd
commit ca59d8149c
No known key found for this signature in database
GPG key ID: 4AEE18F83AFDEB23
55 changed files with 268 additions and 223 deletions

View file

@ -4,8 +4,10 @@
package docs.stream
import akka.Done
import akka.NotUsed
import akka.actor.{ Actor, ActorSystem, Cancellable }
import akka.stream.CompletionStrategy
import akka.stream.Materializer
import akka.stream.{ ClosedShape, FlowShape, OverflowStrategy }
import akka.stream.scaladsl._
@ -99,7 +101,7 @@ class FlowDocSpec extends AkkaSpec with CompileOnlySpec {
Source(List(1, 2, 3))
// Create a source from a Future
Source.fromFuture(Future.successful("Hello Streams!"))
Source.future(Future.successful("Hello Streams!"))
// Create a source from a single element
Source.single("only one element")
@ -227,8 +229,13 @@ class FlowDocSpec extends AkkaSpec with CompileOnlySpec {
"source pre-materialization" in {
//#source-prematerialization
val completeWithDone: PartialFunction[Any, CompletionStrategy] = { case Done => CompletionStrategy.immediately }
val matValuePoweredSource =
Source.actorRef[String](bufferSize = 100, overflowStrategy = OverflowStrategy.fail)
Source.actorRef[String](
completionMatcher = completeWithDone,
failureMatcher = PartialFunction.empty,
bufferSize = 100,
overflowStrategy = OverflowStrategy.fail)
val (actorRef, source) = matValuePoweredSource.preMaterialize()

View file

@ -42,7 +42,7 @@ class FlowStreamRefsDocSpec extends AkkaSpec with CompileOnlySpec {
//#offer-source
//#offer-source-use
val sourceActor = system.actorOf(Props[DataSource], "dataSource")
val sourceActor = system.actorOf(Props[DataSource](), "dataSource")
sourceActor ! RequestLogs(1337)
val offer = expectMsgType[LogsOffer]
@ -87,7 +87,7 @@ class FlowStreamRefsDocSpec extends AkkaSpec with CompileOnlySpec {
def localMetrics(): Source[String, NotUsed] = Source.single("")
//#offer-sink-use
val receiver = system.actorOf(Props[DataReceiver], "receiver")
val receiver = system.actorOf(Props[DataReceiver](), "receiver")
receiver ! PrepareUpload("system-42-tmp")
val ready = expectMsgType[MeasurementsSinkReady]

View file

@ -506,7 +506,7 @@ class GraphStageDocSpec extends AkkaSpec {
if (buffer.isEmpty) {
downstreamWaiting = true
} else {
val elem = buffer.dequeue
val elem = buffer.dequeue()
push(out, elem)
}
if (!bufferFull && !hasBeenPulled(in)) {

View file

@ -4,9 +4,10 @@
package docs.stream
import akka.NotUsed
import scala.concurrent.duration._
import akka.Done
import akka.NotUsed
import akka.testkit.AkkaSpec
import akka.stream.scaladsl._
import akka.stream._
@ -134,7 +135,7 @@ class IntegrationDocSpec extends AkkaSpec(IntegrationDocSpec.config) {
import TwitterStreamQuickstartDocSpec._
import IntegrationDocSpec._
val ref: ActorRef = system.actorOf(Props[Translator])
val ref: ActorRef = system.actorOf(Props[Translator]())
"ask" in {
//#ask
@ -494,16 +495,25 @@ class IntegrationDocSpec extends AkkaSpec(IntegrationDocSpec.config) {
//#source-actorRef
val bufferSize = 10
val cm: PartialFunction[Any, CompletionStrategy] = {
case Done =>
CompletionStrategy.immediately
}
val ref = Source
.actorRef[Int](bufferSize, OverflowStrategy.fail) // note: backpressure is not supported
.actorRef[Int](
completionMatcher = cm,
failureMatcher = PartialFunction.empty[Any, Throwable],
bufferSize = bufferSize,
overflowStrategy = OverflowStrategy.fail) // note: backpressure is not supported
.map(x => x * x)
.toMat(Sink.foreach(x => println(s"completed $x")))(Keep.left)
.toMat(Sink.foreach((x: Int) => println(s"completed $x")))(Keep.left)
.run()
ref ! 1
ref ! 2
ref ! 3
ref ! akka.actor.Status.Success("done")
ref ! Done
//#source-actorRef
}
}

View file

@ -28,7 +28,11 @@ class RateTransformationDocSpec extends AkkaSpec {
//#conflate-summarize
val fut =
Source.fromIterator(() => Iterator.continually(Random.nextGaussian)).via(statsFlow).grouped(10).runWith(Sink.head)
Source
.fromIterator(() => Iterator.continually(Random.nextGaussian()))
.via(statsFlow)
.grouped(10)
.runWith(Sink.head)
fut.futureValue
}
@ -38,8 +42,8 @@ class RateTransformationDocSpec extends AkkaSpec {
val p = 0.01
val sampleFlow = Flow[Double]
.conflateWithSeed(immutable.Seq(_)) {
case (acc, elem) if Random.nextDouble < p => acc :+ elem
case (acc, _) => acc
case (acc, elem) if Random.nextDouble() < p => acc :+ elem
case (acc, _) => acc
}
.mapConcat(identity)
//#conflate-sample

View file

@ -41,9 +41,9 @@ class ReactiveStreamsDocSpec extends AkkaSpec {
override def tweets: Publisher[Tweet] =
TwitterStreamQuickstartDocSpec.tweets.runWith(Sink.asPublisher(fanout = false))
override def storage = TestSubscriber.manualProbe[Author]
override def storage = TestSubscriber.manualProbe[Author]()
override def alert = TestSubscriber.manualProbe[Author]
override def alert = TestSubscriber.manualProbe[Author]()
}
def assertResult(storage: TestSubscriber.ManualProbe[Author]): Unit = {

View file

@ -41,7 +41,7 @@ class RestartDocSpec extends AkkaSpec with CompileOnlySpec {
maxRestarts = 20 // limits the amount of restarts to 20
) { () =>
// Create a source from a future of a source
Source.fromFutureSource {
Source.futureSource {
// Make a single request with akka-http
Http()
.singleRequest(HttpRequest(uri = "http://example.com/eventstream"))

View file

@ -114,7 +114,7 @@ class StreamPartialGraphDSLDocSpec extends AkkaSpec {
"combine sinks with simplified API" in {
val actorRef: ActorRef = testActor
//#sink-combine
val sendRmotely = Sink.actorRef(actorRef, "Done")
val sendRmotely = Sink.actorRef(actorRef, "Done", _ => "Failed")
val localProcessing = Sink.foreach[Int](_ => /* do something useful */ ())
val sink = Sink.combine(sendRmotely, localProcessing)(Broadcast[Int](_))

View file

@ -4,12 +4,14 @@
package docs.stream
import akka.stream._
import akka.stream.scaladsl._
import akka.stream.testkit.scaladsl._
import scala.util._
import scala.concurrent.duration._
import scala.concurrent._
import akka.Done
import akka.stream._
import akka.stream.scaladsl._
import akka.stream.testkit.scaladsl._
import akka.testkit.{ AkkaSpec, TestProbe }
import akka.pattern
@ -67,10 +69,12 @@ class StreamTestKitDocSpec extends AkkaSpec {
val sourceUnderTest = Source.tick(0.seconds, 200.millis, Tick)
val probe = TestProbe()
val cancellable = sourceUnderTest.to(Sink.actorRef(probe.ref, "completed")).run()
val cancellable = sourceUnderTest
.to(Sink.actorRef(probe.ref, onCompleteMessage = "completed", onFailureMessage = _ => "failed"))
.run()
probe.expectMsg(1.second, Tick)
probe.expectNoMsg(100.millis)
probe.expectNoMessage(100.millis)
probe.expectMsg(3.seconds, Tick)
cancellable.cancel()
probe.expectMsg(3.seconds, "completed")
@ -81,12 +85,23 @@ class StreamTestKitDocSpec extends AkkaSpec {
//#source-actorref
val sinkUnderTest = Flow[Int].map(_.toString).toMat(Sink.fold("")(_ + _))(Keep.right)
val (ref, future) = Source.actorRef(8, OverflowStrategy.fail).toMat(sinkUnderTest)(Keep.both).run()
val (ref, future) = Source
.actorRef(
completionMatcher = {
case Done =>
CompletionStrategy.draining
},
// Never fail the stream because of a message:
failureMatcher = PartialFunction.empty,
bufferSize = 8,
overflowStrategy = OverflowStrategy.fail)
.toMat(sinkUnderTest)(Keep.both)
.run()
ref ! 1
ref ! 2
ref ! 3
ref ! akka.actor.Status.Success(CompletionStrategy.draining)
ref ! Done
val result = Await.result(future, 3.seconds)
assert(result == "123")
@ -116,9 +131,7 @@ class StreamTestKitDocSpec extends AkkaSpec {
val (probe, future) = TestSource.probe[Int].toMat(sinkUnderTest)(Keep.both).run()
probe.sendError(new Exception("boom"))
Await.ready(future, 3.seconds)
val Failure(exception) = future.value.get
assert(exception.getMessage == "boom")
assert(future.failed.futureValue.getMessage == "boom")
//#injecting-failure
}

View file

@ -18,13 +18,13 @@ class RecipeAdhocSource extends RecipeSpec {
//#adhoc-source
def adhocSource[T](source: Source[T, _], timeout: FiniteDuration, maxRetries: Int): Source[T, _] =
Source.lazily(
Source.lazySource(
() =>
source
.backpressureTimeout(timeout)
.recoverWithRetries(maxRetries, {
case t: TimeoutException =>
Source.lazily(() => source.backpressureTimeout(timeout)).mapMaterializedValue(_ => NotUsed)
Source.lazySource(() => source.backpressureTimeout(timeout)).mapMaterializedValue(_ => NotUsed)
}))
//#adhoc-source

View file

@ -4,7 +4,6 @@
package docs.stream.operators
import akka.Done
import akka.actor.ActorSystem
import akka.testkit.TestProbe
@ -12,7 +11,7 @@ object SourceOperators {
implicit val system: ActorSystem = ???
def fromFuture = {
def fromFuture(): Unit = {
//#sourceFromFuture
import akka.actor.ActorSystem
@ -35,7 +34,6 @@ object SourceOperators {
import akka.stream.OverflowStrategy
import akka.stream.CompletionStrategy
import akka.stream.scaladsl._
import scala.util.Failure
val source: Source[Any, ActorRef] = Source.actorRef(
completionMatcher = {

View file

@ -36,10 +36,10 @@ class StreamConvertersToJava extends AkkaSpec with Futures {
"demonstrate conversion from Java8 streams" in {
//#fromJavaStream
def factory(): IntStream = IntStream.rangeClosed(0, 9)
val source: Source[Int, NotUsed] = StreamConverters.fromJavaStream(factory).map(_.intValue())
val source: Source[Int, NotUsed] = StreamConverters.fromJavaStream(() => factory()).map(_.intValue())
val sink: Sink[Int, Future[immutable.Seq[Int]]] = Sink.seq[Int]
val futureInts: Future[immutable.Seq[Int]] = source.toMat(sink)(Keep.right).run
val futureInts: Future[immutable.Seq[Int]] = source.toMat(sink)(Keep.right).run()
//#fromJavaStream
whenReady(futureInts) { ints =>

View file

@ -16,9 +16,6 @@ import akka.stream.scaladsl.Source
import akka.stream.scaladsl.Tcp
import akka.stream.testkit.TestPublisher
import akka.stream.testkit.TestSubscriber
import akka.stream.testkit.Utils.TE
import akka.stream.testkit.scaladsl.TestSource
import akka.stream.testkit.scaladsl.TestSink
import akka.util.ByteString
import scala.concurrent.duration._
@ -63,7 +60,7 @@ object FromSinkAndSource {
def testing(): Unit = {
def myApiThatTakesAFlow[In, Out](flow: Flow[In, Out, NotUsed]): Unit = ???
// #testing
val inProbe = TestSubscriber.probe[String]
val inProbe = TestSubscriber.probe[String]()
val outProbe = TestPublisher.probe[String]()
val testFlow = Flow.fromSinkAndSource(Sink.fromSubscriber(inProbe), Source.fromPublisher(outProbe))

View file

@ -52,7 +52,7 @@ object Restart extends App {
}
def onRestartWithBackoffInnerComplete() {
def onRestartWithBackoffInnerComplete() = {
//#restart-failure-inner-complete
val finiteSource = Source.tick(1.second, 1.second, "tick").take(3)

View file

@ -27,7 +27,7 @@ object Tick {
case class Response(text: String)
}
def simple() {
def simple() = {
// #simple
Source
.tick(
@ -39,7 +39,7 @@ object Tick {
// #simple
}
def pollSomething() {
def pollSomething() = {
// #poll-actor
val periodicActorResponse: Source[String, Cancellable] = Source
.tick(1.second, 1.second, "tick")

View file

@ -4,7 +4,6 @@
package docs.stream.operators.source
import akka.NotUsed
import akka.actor.typed.ActorSystem
import akka.stream.scaladsl.Source
@ -48,7 +47,7 @@ object Zip {
// #zipWithN-simple
}
def zipAll() {
def zipAll() = {
// #zipAll-simple
val numbers = Source(1 :: 2 :: 3 :: 4 :: Nil)
val letters = Source("a" :: "b" :: "c" :: Nil)