Make warnings in Java code fatal (#28402)

This commit is contained in:
Arnout Engelen 2020-08-04 13:47:38 +02:00 committed by GitHub
parent 58fa1e3604
commit 327e16980d
No known key found for this signature in database
GPG key ID: 4AEE18F83AFDEB23
51 changed files with 480 additions and 213 deletions

View file

@ -13,6 +13,7 @@ import java.util.stream.IntStream;
import org.junit.Test; import org.junit.Test;
import akka.japi.Creator; import akka.japi.Creator;
import org.scalatestplus.junit.JUnitSuite; import org.scalatestplus.junit.JUnitSuite;
public class ActorCreationTest extends JUnitSuite { public class ActorCreationTest extends JUnitSuite {
@ -77,6 +78,7 @@ public class ActorCreationTest extends JUnitSuite {
return Props.create(TestActor.class, () -> new TestActor(magicNumber)); return Props.create(TestActor.class, () -> new TestActor(magicNumber));
} }
@Deprecated
public static Props propsUsingLamdaWithoutClass(Integer magicNumber) { public static Props propsUsingLamdaWithoutClass(Integer magicNumber) {
return Props.create(() -> new TestActor(magicNumber)); return Props.create(() -> new TestActor(magicNumber));
} }
@ -113,6 +115,7 @@ public class ActorCreationTest extends JUnitSuite {
public static Props propsUsingCreatorWithoutClass(final int magicNumber) { public static Props propsUsingCreatorWithoutClass(final int magicNumber) {
return Props.create( return Props.create(
TestActor2.class,
new Creator<TestActor2>() { new Creator<TestActor2>() {
private static final long serialVersionUID = 1L; private static final long serialVersionUID = 1L;
@ -135,7 +138,7 @@ public class ActorCreationTest extends JUnitSuite {
public static Props propsUsingStaticCreator(final int magicNumber) { public static Props propsUsingStaticCreator(final int magicNumber) {
return Props.create(staticCreator); return Props.create(TestActor2.class, staticCreator);
} }
final int magicNumber; final int magicNumber;
@ -174,6 +177,7 @@ public class ActorCreationTest extends JUnitSuite {
public void testWrongAnonymousInPlaceCreator() { public void testWrongAnonymousInPlaceCreator() {
try { try {
Props.create( Props.create(
Actor.class,
new Creator<Actor>() { new Creator<Actor>() {
@Override @Override
public Actor create() throws Exception { public Actor create() throws Exception {
@ -190,6 +194,7 @@ public class ActorCreationTest extends JUnitSuite {
@Test @Test
@SuppressWarnings("unchecked") @SuppressWarnings("unchecked")
@Deprecated
public void testWrongErasedStaticCreator() { public void testWrongErasedStaticCreator() {
try { try {
Props.create(new G()); Props.create(new G());
@ -202,6 +207,7 @@ public class ActorCreationTest extends JUnitSuite {
Props.create(AbstractActor.class, new G()); Props.create(AbstractActor.class, new G());
} }
@Deprecated
@Test @Test
public void testRightStaticCreator() { public void testRightStaticCreator() {
final Props p = Props.create(new C()); final Props p = Props.create(new C());
@ -209,6 +215,7 @@ public class ActorCreationTest extends JUnitSuite {
} }
@Test @Test
@Deprecated
public void testWrongAnonymousClassStaticCreator() { public void testWrongAnonymousClassStaticCreator() {
try { try {
Props.create(new C() {}); // has implicit reference to outer class Props.create(new C() {}); // has implicit reference to outer class
@ -223,23 +230,26 @@ public class ActorCreationTest extends JUnitSuite {
@Test @Test
public void testRightTopLevelNonStaticCreator() { public void testRightTopLevelNonStaticCreator() {
final Creator<UntypedAbstractActor> nonStatic = new NonStaticCreator(); final Creator<UntypedAbstractActor> nonStatic = new NonStaticCreator();
final Props p = Props.create(nonStatic); final Props p = Props.create(UntypedAbstractActor.class, nonStatic);
assertEquals(UntypedAbstractActor.class, p.actorClass()); assertEquals(UntypedAbstractActor.class, p.actorClass());
} }
@Test @Test
@Deprecated
public void testRightStaticParametricCreator() { public void testRightStaticParametricCreator() {
final Props p = Props.create(new D<AbstractActor>()); final Props p = Props.create(new D<AbstractActor>());
assertEquals(Actor.class, p.actorClass()); assertEquals(Actor.class, p.actorClass());
} }
@Test @Test
@Deprecated
public void testRightStaticBoundedCreator() { public void testRightStaticBoundedCreator() {
final Props p = Props.create(new E<AbstractActor>()); final Props p = Props.create(new E<AbstractActor>());
assertEquals(AbstractActor.class, p.actorClass()); assertEquals(AbstractActor.class, p.actorClass());
} }
@Test @Test
@Deprecated
public void testRightStaticSuperinterface() { public void testRightStaticSuperinterface() {
final Props p = Props.create(new F()); final Props p = Props.create(new F());
assertEquals(AbstractActor.class, p.actorClass()); assertEquals(AbstractActor.class, p.actorClass());
@ -267,6 +277,7 @@ public class ActorCreationTest extends JUnitSuite {
} }
@Test @Test
@Deprecated
public void testAnonymousClassCreatedInStaticMethodCreator() { public void testAnonymousClassCreatedInStaticMethodCreator() {
final Creator<AbstractActor> anonymousCreatorFromStaticMethod = final Creator<AbstractActor> anonymousCreatorFromStaticMethod =
createAnonymousCreatorInStaticMethod(); createAnonymousCreatorInStaticMethod();
@ -274,12 +285,14 @@ public class ActorCreationTest extends JUnitSuite {
} }
@Test @Test
@Deprecated
public void testClassCreatorWithArguments() { public void testClassCreatorWithArguments() {
final Creator<AbstractActor> anonymousCreatorFromStaticMethod = new P("hello"); final Creator<AbstractActor> anonymousCreatorFromStaticMethod = new P("hello");
Props.create(anonymousCreatorFromStaticMethod); Props.create(anonymousCreatorFromStaticMethod);
} }
@Test @Test
@Deprecated
public void testAnonymousClassCreatorWithArguments() { public void testAnonymousClassCreatorWithArguments() {
try { try {
final Creator<AbstractActor> anonymousCreatorFromStaticMethod = new P("hello") { final Creator<AbstractActor> anonymousCreatorFromStaticMethod = new P("hello") {
@ -301,6 +314,7 @@ public class ActorCreationTest extends JUnitSuite {
} }
@Test @Test
@Deprecated
public void testWrongPropsUsingLambdaWithoutClass() { public void testWrongPropsUsingLambdaWithoutClass() {
final Props p = TestActor.propsUsingLamda(17); final Props p = TestActor.propsUsingLamda(17);
assertEquals(TestActor.class, p.actorClass()); assertEquals(TestActor.class, p.actorClass());
@ -330,12 +344,12 @@ public class ActorCreationTest extends JUnitSuite {
public void testIssue20537Reproducer() { public void testIssue20537Reproducer() {
final Issue20537Reproducer.ReproducerCreator creator = final Issue20537Reproducer.ReproducerCreator creator =
new Issue20537Reproducer.ReproducerCreator(false); new Issue20537Reproducer.ReproducerCreator(false);
final Props p = Props.create(creator); final Props p = Props.create(Issue20537Reproducer.class, creator);
assertEquals(Issue20537Reproducer.class, p.actorClass()); assertEquals(Issue20537Reproducer.class, p.actorClass());
ArrayList<Props> pList = ArrayList<Props> pList =
IntStream.range(0, 4) IntStream.range(0, 4)
.mapToObj(i -> Props.create(creator)) .mapToObj(i -> Props.create(Issue20537Reproducer.class, creator))
.collect(toCollection(ArrayList::new)); .collect(toCollection(ArrayList::new));
for (Props each : pList) { for (Props each : pList) {
assertEquals(Issue20537Reproducer.class, each.actorClass()); assertEquals(Issue20537Reproducer.class, each.actorClass());

View file

@ -4,17 +4,17 @@
package akka.actor; package akka.actor;
import java.time.Duration;
import java.util.concurrent.CompletionStage;
import java.util.concurrent.TimeUnit;
import akka.testkit.AkkaJUnitActorSystemResource; import akka.testkit.AkkaJUnitActorSystemResource;
import akka.testkit.AkkaSpec; import akka.testkit.AkkaSpec;
import org.junit.ClassRule; import org.junit.ClassRule;
import org.junit.Test; import org.junit.Test;
import org.scalatestplus.junit.JUnitSuite; import org.scalatestplus.junit.JUnitSuite;
import java.time.Duration;
import java.util.concurrent.CompletionStage;
import java.util.concurrent.TimeUnit;
import static org.junit.Assert.assertEquals; import static org.junit.Assert.assertEquals;
public class ActorSelectionTest extends JUnitSuite { public class ActorSelectionTest extends JUnitSuite {

View file

@ -91,7 +91,7 @@ public class JavaAPI extends JUnitSuite {
public void mustBeAbleToCreateActorWIthConstructorParams() { public void mustBeAbleToCreateActorWIthConstructorParams() {
ActorRef ref = ActorRef ref =
system.actorOf( system.actorOf(
Props.create(ActorWithConstructorParams.class, "a", "b", new Integer(17), 18)); Props.create(ActorWithConstructorParams.class, "a", "b", Integer.valueOf(17), 18));
final TestProbe probe = new TestProbe(system); final TestProbe probe = new TestProbe(system);
probe.send(ref, "get"); probe.send(ref, "get");
probe.expectMsg("a-b-17-18"); probe.expectMsg("a-b-17-18");
@ -101,7 +101,7 @@ public class JavaAPI extends JUnitSuite {
public void mustBeAbleToCreateActorWIthBoxedAndUnBoxedConstructorParams() { public void mustBeAbleToCreateActorWIthBoxedAndUnBoxedConstructorParams() {
ActorRef ref = ActorRef ref =
system.actorOf( system.actorOf(
Props.create(ActorWithConstructorParams.class, "a", "b", 17, new Integer(18))); Props.create(ActorWithConstructorParams.class, "a", "b", 17, Integer.valueOf(18)));
final TestProbe probe = new TestProbe(system); final TestProbe probe = new TestProbe(system);
probe.send(ref, "get"); probe.send(ref, "get");
probe.expectMsg("a-b-17-18"); probe.expectMsg("a-b-17-18");

View file

@ -16,7 +16,7 @@ public class StashJavaAPITestActors {
Object msg, int count, ActorRef sender, ActorRef self, UnrestrictedStash stash) { Object msg, int count, ActorRef sender, ActorRef self, UnrestrictedStash stash) {
if (msg instanceof String) { if (msg instanceof String) {
if (count < 0) { if (count < 0) {
sender.tell(new Integer(((String) msg).length()), self); sender.tell(Integer.valueOf(((String) msg).length()), self);
} else if (count == 2) { } else if (count == 2) {
stash.unstashAll(); stash.unstashAll();
return -1; return -1;

View file

@ -120,11 +120,11 @@ public class LoggingAdapterTest extends JUnitSuite {
*/ */
@Test @Test
public void mustBeAbleToCreateLogEventsWithOldConstructor() throws Exception { public void mustBeAbleToCreateLogEventsWithOldConstructor() throws Exception {
assertNotNull(new Error(new Exception(), "logSource", this.getClass(), "The message")); assertNotNull(new Error(new Exception(), "logSource", LoggingAdapterTest.class, "The message"));
assertNotNull(new Error("logSource", this.getClass(), "The message")); assertNotNull(new Error("logSource", LoggingAdapterTest.class, "The message"));
assertNotNull(new Warning("logSource", this.getClass(), "The message")); assertNotNull(new Warning("logSource", LoggingAdapterTest.class, "The message"));
assertNotNull(new Info("logSource", this.getClass(), "The message")); assertNotNull(new Info("logSource", LoggingAdapterTest.class, "The message"));
assertNotNull(new Debug("logSource", this.getClass(), "The message")); assertNotNull(new Debug("logSource", LoggingAdapterTest.class, "The message"));
} }
private static class LogJavaTestKit extends TestKit { private static class LogJavaTestKit extends TestKit {

View file

@ -7,6 +7,8 @@ package akka.japi;
import akka.japi.pf.FI; import akka.japi.pf.FI;
import akka.japi.pf.Match; import akka.japi.pf.Match;
import org.junit.Rule; import org.junit.Rule;
import org.junit.Assert;
import org.junit.function.ThrowingRunnable;
import org.junit.rules.ExpectedException; import org.junit.rules.ExpectedException;
import org.junit.Test; import org.junit.Test;
import org.scalatestplus.junit.JUnitSuite; import org.scalatestplus.junit.JUnitSuite;
@ -16,8 +18,6 @@ import static org.junit.Assert.*;
public class MatchBuilderTest extends JUnitSuite { public class MatchBuilderTest extends JUnitSuite {
@Rule public ExpectedException exception = ExpectedException.none();
@Test @Test
public void shouldPassBasicMatchTest() { public void shouldPassBasicMatchTest() {
Match<Object, Double> pf = Match<Object, Double> pf =
@ -41,13 +41,19 @@ public class MatchBuilderTest extends JUnitSuite {
assertTrue( assertTrue(
"An integer should be multiplied by 10", "An integer should be multiplied by 10",
new Double(47110).equals(pf.match(new Integer(4711)))); Double.valueOf(47110).equals(pf.match(Integer.valueOf(4711))));
assertTrue( assertTrue(
"A double should be multiplied by -10", "A double should be multiplied by -10",
new Double(-47110).equals(pf.match(new Double(4711)))); Double.valueOf(-47110).equals(pf.match(Double.valueOf(4711))));
exception.expect(MatchError.class); Assert.assertThrows(
assertFalse("A string should throw a MatchError", new Double(4711).equals(pf.match("4711"))); "A string should throw a MatchError",
MatchError.class,
new ThrowingRunnable() {
public void run() {
pf.match("4711");
}
});
} }
static class GenericClass<T> { static class GenericClass<T> {
@ -95,9 +101,13 @@ public class MatchBuilderTest extends JUnitSuite {
} }
})); }));
exception.expect(MatchError.class); Assert.assertThrows(
assertTrue(
"empty GenericMessage should throw match error", "empty GenericMessage should throw match error",
"".equals(pf.match(new GenericClass<String>("")))); MatchError.class,
new ThrowingRunnable() {
public void run() {
pf.match(new GenericClass<String>(""));
}
});
} }
} }

View file

@ -8,6 +8,7 @@ import java.util.Arrays;
import java.util.List; import java.util.List;
import org.junit.Test; import org.junit.Test;
import org.junit.Before; import org.junit.Before;
import org.scalatestplus.junit.JUnitSuite;
import akka.actor.AbstractActor.Receive; import akka.actor.AbstractActor.Receive;
import org.scalatestplus.junit.JUnitSuite; import org.scalatestplus.junit.JUnitSuite;

View file

@ -10,7 +10,6 @@ import scala.concurrent.duration._
import com.github.ghik.silencer.silent import com.github.ghik.silencer.silent
import language.postfixOps import language.postfixOps
import org.scalatest.BeforeAndAfterEach import org.scalatest.BeforeAndAfterEach
import org.scalatestplus.junit.JUnitSuiteLike
import akka.pattern.ask import akka.pattern.ask
import akka.testkit._ import akka.testkit._
@ -98,9 +97,6 @@ object ActorWithStashSpec {
} }
@silent
class JavaActorWithStashSpec extends StashJavaAPI with JUnitSuiteLike
@silent @silent
class ActorWithStashSpec extends AkkaSpec with DefaultTimeout with BeforeAndAfterEach { class ActorWithStashSpec extends AkkaSpec with DefaultTimeout with BeforeAndAfterEach {
import ActorWithStashSpec._ import ActorWithStashSpec._

View file

@ -8,16 +8,11 @@ import java.util.concurrent.atomic.AtomicInteger
import scala.util.control.NoStackTrace import scala.util.control.NoStackTrace
import com.github.ghik.silencer.silent import akka.testkit.EventFilter
import akka.testkit.TestKit._
import com.typesafe.config.ConfigFactory import com.typesafe.config.ConfigFactory
import org.scalatest.matchers.should.Matchers import org.scalatest.matchers.should.Matchers
import org.scalatest.wordspec.AnyWordSpec import org.scalatest.wordspec.AnyWordSpec
import org.scalatestplus.junit.JUnitSuiteLike
import akka.testkit.EventFilter
import akka.testkit.TestKit._
@silent
class JavaExtensionSpec extends JavaExtension with JUnitSuiteLike
object TestExtension extends ExtensionId[TestExtension] with ExtensionIdProvider { object TestExtension extends ExtensionId[TestExtension] with ExtensionIdProvider {
def lookup = this def lookup = this

View file

@ -1,11 +0,0 @@
/*
* Copyright (C) 2009-2020 Lightbend Inc. <https://www.lightbend.com>
*/
package akka.actor
import com.github.ghik.silencer.silent
import org.scalatestplus.junit.JUnitSuiteLike
@silent
class JavaAPISpec extends JavaAPI with JUnitSuiteLike

View file

@ -1,11 +0,0 @@
/*
* Copyright (C) 2018-2020 Lightbend Inc. <https://www.lightbend.com>
*/
package akka.japi
import com.github.ghik.silencer.silent
import org.scalatestplus.junit.JUnitSuiteLike
@silent
class JavaAPITest extends JavaAPITestBase with JUnitSuiteLike

View file

@ -1,11 +0,0 @@
/*
* Copyright (C) 2009-2020 Lightbend Inc. <https://www.lightbend.com>
*/
package akka.util
import com.github.ghik.silencer.silent
import org.scalatestplus.junit.JUnitSuiteLike
@silent
class JavaDurationSpec extends JavaDuration with JUnitSuiteLike

View file

@ -78,6 +78,7 @@ public class Aggregator<Reply, Aggregate> extends AbstractBehavior<Aggregator.Co
} }
@Override @Override
@SuppressWarnings("unchecked")
public Receive<Command> createReceive() { public Receive<Command> createReceive() {
return newReceiveBuilder() return newReceiveBuilder()
.onMessage(WrappedReply.class, this::onReply) .onMessage(WrappedReply.class, this::onReply)

View file

@ -91,6 +91,7 @@ public class TailChopping<Reply> extends AbstractBehavior<TailChopping.Command>
} }
@Override @Override
@SuppressWarnings("unchecked")
public Receive<Command> createReceive() { public Receive<Command> createReceive() {
return newReceiveBuilder() return newReceiveBuilder()
.onMessage(WrappedReply.class, this::onReply) .onMessage(WrappedReply.class, this::onReply)

View file

@ -8,7 +8,6 @@ import akka.util.JavaDurationConverters;
import scala.concurrent.ExecutionContext; import scala.concurrent.ExecutionContext;
import scala.concurrent.duration.FiniteDuration; import scala.concurrent.duration.FiniteDuration;
// #scheduler
/** /**
* An Akka scheduler service. This one needs one special behavior: if Closeable, it MUST execute all * An Akka scheduler service. This one needs one special behavior: if Closeable, it MUST execute all
* outstanding tasks upon .close() in order to properly shutdown all dispatchers. * outstanding tasks upon .close() in order to properly shutdown all dispatchers.
@ -30,6 +29,7 @@ public abstract class AbstractScheduler extends AbstractSchedulerBase {
* would like the function to be run after 2 seconds and thereafter every 100ms you would set * would like the function to be run after 2 seconds and thereafter every 100ms you would set
* delay = Duration(2, TimeUnit.SECONDS) and interval = Duration(100, TimeUnit.MILLISECONDS) * delay = Duration(2, TimeUnit.SECONDS) and interval = Duration(100, TimeUnit.MILLISECONDS)
*/ */
@Deprecated
@Override @Override
public abstract Cancellable schedule( public abstract Cancellable schedule(
FiniteDuration initialDelay, FiniteDuration initialDelay,
@ -42,6 +42,7 @@ public abstract class AbstractScheduler extends AbstractSchedulerBase {
* would like the function to be run after 2 seconds and thereafter every 100ms you would set * would like the function to be run after 2 seconds and thereafter every 100ms you would set
* delay = Duration(2, TimeUnit.SECONDS) and interval = Duration.ofMillis(100) * delay = Duration(2, TimeUnit.SECONDS) and interval = Duration.ofMillis(100)
*/ */
@Deprecated
public Cancellable schedule( public Cancellable schedule(
final java.time.Duration initialDelay, final java.time.Duration initialDelay,
final java.time.Duration interval, final java.time.Duration interval,
@ -78,4 +79,3 @@ public abstract class AbstractScheduler extends AbstractSchedulerBase {
@Override @Override
public abstract double maxFrequency(); public abstract double maxFrequency();
} }
// #scheduler

View file

@ -38,6 +38,7 @@ public class ClusterClientTest extends JUnitSuite {
// #initialContacts // #initialContacts
@Test @Test
@Deprecated
public void demonstrateUsage() { public void demonstrateUsage() {
// #server // #server
ActorRef serviceA = system.actorOf(Props.create(Service.class), "serviceA"); ActorRef serviceA = system.actorOf(Props.create(Service.class), "serviceA");

View file

@ -524,14 +524,17 @@ public class ActorDocTest extends AbstractJavaTest {
// #creating-props-deprecated // #creating-props-deprecated
} }
@Test(expected = IllegalArgumentException.class) // Commented out because this 'Props.create' overload is now deprecated
// @Test(expected = IllegalArgumentException.class)
public void creatingPropsIllegal() { public void creatingPropsIllegal() {
/*
// #creating-props-illegal // #creating-props-illegal
// This will throw an IllegalArgumentException since some runtime // This will throw an IllegalArgumentException since some runtime
// type information of the lambda is erased. // type information of the lambda is erased.
// Use Props.create(actorClass, Creator) instead. // Use Props.create(actorClass, Creator) instead.
Props props = Props.create(() -> new ActorWithArgs("arg")); Props props = Props.create(() -> new ActorWithArgs("arg"));
// #creating-props-illegal // #creating-props-illegal
*/
} }
public public

View file

@ -24,14 +24,27 @@ public class DnsCompileOnlyDocTest {
final Duration timeout = Duration.ofMillis(1000L); final Duration timeout = Duration.ofMillis(1000L);
// #resolve // #resolve
Option<Dns.Resolved> initial = Dns.get(system).cache().resolve("google.com", system, actorRef); Option<DnsProtocol.Resolved> initial =
Option<Dns.Resolved> cached = Dns.get(system).cache().cached("google.com"); Dns.get(system)
.cache()
.resolve(
new DnsProtocol.Resolve("google.com", DnsProtocol.ipRequestType()),
system,
actorRef);
Option<DnsProtocol.Resolved> cached =
Dns.get(system)
.cache()
.cached(new DnsProtocol.Resolve("google.com", DnsProtocol.ipRequestType()));
// #resolve // #resolve
{ {
// #actor-api-inet-address // #actor-api-inet-address
final ActorRef dnsManager = Dns.get(system).manager(); final ActorRef dnsManager = Dns.get(system).manager();
CompletionStage<Object> resolved = ask(dnsManager, new Dns.Resolve("google.com"), timeout); CompletionStage<Object> resolved =
ask(
dnsManager,
new DnsProtocol.Resolve("google.com", DnsProtocol.ipRequestType()),
timeout);
// #actor-api-inet-address // #actor-api-inet-address
} }

View file

@ -21,6 +21,7 @@ import akka.cluster.ddata.Replicator.Changed;
import akka.cluster.ddata.Replicator.Subscribe; import akka.cluster.ddata.Replicator.Subscribe;
import akka.cluster.ddata.Replicator.Update; import akka.cluster.ddata.Replicator.Update;
import akka.cluster.ddata.Replicator.UpdateResponse; import akka.cluster.ddata.Replicator.UpdateResponse;
import akka.cluster.ddata.SelfUniqueAddress;
import akka.event.Logging; import akka.event.Logging;
import akka.event.LoggingAdapter; import akka.event.LoggingAdapter;
@ -31,7 +32,8 @@ public class DataBot extends AbstractActor {
private final LoggingAdapter log = Logging.getLogger(getContext().getSystem(), this); private final LoggingAdapter log = Logging.getLogger(getContext().getSystem(), this);
private final ActorRef replicator = DistributedData.get(getContext().getSystem()).replicator(); private final ActorRef replicator = DistributedData.get(getContext().getSystem()).replicator();
private final Cluster node = Cluster.get(getContext().getSystem()); private final SelfUniqueAddress node =
DistributedData.get(getContext().getSystem()).selfUniqueAddress();
private final Cancellable tickTask = private final Cancellable tickTask =
getContext() getContext()

View file

@ -127,7 +127,7 @@ public class DistributedDataDocTest extends AbstractJavaTest {
static static
// #update-request-context // #update-request-context
class DemonstrateUpdateWithRequestContext extends AbstractActor { class DemonstrateUpdateWithRequestContext extends AbstractActor {
final Cluster node = Cluster.get(getContext().getSystem()); final SelfUniqueAddress node = DistributedData.get(system).selfUniqueAddress();
final ActorRef replicator = DistributedData.get(getContext().getSystem()).replicator(); final ActorRef replicator = DistributedData.get(getContext().getSystem()).replicator();
final WriteConsistency writeTwo = new WriteTo(2, Duration.ofSeconds(3)); final WriteConsistency writeTwo = new WriteTo(2, Duration.ofSeconds(3));

View file

@ -71,7 +71,7 @@ public class MyEventsByTagSource extends GraphStage<SourceShape<EventEnvelope>>
@Override @Override
public void preStart() { public void preStart() {
schedulePeriodically(Continue.INSTANCE, refreshInterval); scheduleWithFixedDelay(Continue.INSTANCE, refreshInterval, refreshInterval);
} }
@Override @Override
@ -109,7 +109,8 @@ public class MyEventsByTagSource extends GraphStage<SourceShape<EventEnvelope>>
Offset.sequence(currentOffset), Offset.sequence(currentOffset),
rs.getString("persistence_id"), rs.getString("persistence_id"),
rs.getLong("seq_nr"), rs.getLong("seq_nr"),
deserialized)); deserialized,
System.currentTimeMillis()));
} }
buf = res; buf = res;
} }

View file

@ -4,6 +4,7 @@
package jdocs.stream; package jdocs.stream;
import akka.Done;
import akka.NotUsed; import akka.NotUsed;
import akka.actor.AbstractActor; import akka.actor.AbstractActor;
import akka.actor.ActorRef; import akka.actor.ActorRef;
@ -157,7 +158,7 @@ public class FlowDocTest extends AbstractJavaTest {
Source.from(list); Source.from(list);
// Create a source form a Future // Create a source form a Future
Source.fromFuture(Futures.successful("Hello Streams!")); Source.future(Futures.successful("Hello Streams!"));
// Create a source from a single element // Create a source from a single element
Source.single("only one element"); Source.single("only one element");
@ -274,7 +275,17 @@ public class FlowDocTest extends AbstractJavaTest {
@Test @Test
public void sourcePreMaterialization() { public void sourcePreMaterialization() {
// #source-prematerialization // #source-prematerialization
Source<String, ActorRef> matValuePoweredSource = Source.actorRef(100, OverflowStrategy.fail()); Source<String, ActorRef> matValuePoweredSource =
Source.actorRef(
elem -> {
// complete stream immediately if we send it Done
if (elem == Done.done()) return Optional.of(CompletionStrategy.immediately());
else return Optional.empty();
},
// never fail the stream because of a message
elem -> Optional.empty(),
100,
OverflowStrategy.fail());
Pair<ActorRef, Source<String, NotUsed>> actorRefSourcePair = Pair<ActorRef, Source<String, NotUsed>> actorRefSourcePair =
matValuePoweredSource.preMaterialize(system); matValuePoweredSource.preMaterialize(system);

View file

@ -179,7 +179,9 @@ public class FlowErrorDocTest extends AbstractJavaTest {
}) })
.recoverWithRetries( .recoverWithRetries(
1, // max attempts 1, // max attempts
new PFBuilder().match(RuntimeException.class, ex -> planB).build()) new PFBuilder<Throwable, Source<String, NotUsed>>()
.match(RuntimeException.class, ex -> planB)
.build())
.runForeach(System.out::println, system); .runForeach(System.out::println, system);
// #recoverWithRetries // #recoverWithRetries

View file

@ -221,7 +221,7 @@ public class GraphStageDocTest extends AbstractJavaTest {
.via(stringLength) .via(stringLength)
.runFold(0, (sum, n) -> sum + n, system); .runFold(0, (sum, n) -> sum + n, system);
assertEquals(new Integer(11), result.toCompletableFuture().get(3, TimeUnit.SECONDS)); assertEquals(Integer.valueOf(11), result.toCompletableFuture().get(3, TimeUnit.SECONDS));
} }
// #many-to-one // #many-to-one
@ -286,7 +286,7 @@ public class GraphStageDocTest extends AbstractJavaTest {
.via(evenFilter) .via(evenFilter)
.runFold(0, (elem, sum) -> sum + elem, system); .runFold(0, (elem, sum) -> sum + elem, system);
assertEquals(new Integer(12), result.toCompletableFuture().get(3, TimeUnit.SECONDS)); assertEquals(Integer.valueOf(12), result.toCompletableFuture().get(3, TimeUnit.SECONDS));
} }
// #one-to-many // #one-to-many
@ -356,7 +356,7 @@ public class GraphStageDocTest extends AbstractJavaTest {
CompletionStage<Integer> result = CompletionStage<Integer> result =
Source.from(Arrays.asList(1, 2, 3)).via(duplicator).runFold(0, (n, sum) -> n + sum, system); Source.from(Arrays.asList(1, 2, 3)).via(duplicator).runFold(0, (n, sum) -> n + sum, system);
assertEquals(new Integer(12), result.toCompletableFuture().get(3, TimeUnit.SECONDS)); assertEquals(Integer.valueOf(12), result.toCompletableFuture().get(3, TimeUnit.SECONDS));
} }
// #simpler-one-to-many // #simpler-one-to-many
@ -412,7 +412,7 @@ public class GraphStageDocTest extends AbstractJavaTest {
CompletionStage<Integer> result = CompletionStage<Integer> result =
Source.from(Arrays.asList(1, 2, 3)).via(duplicator).runFold(0, (n, sum) -> n + sum, system); Source.from(Arrays.asList(1, 2, 3)).via(duplicator).runFold(0, (n, sum) -> n + sum, system);
assertEquals(new Integer(12), result.toCompletableFuture().get(3, TimeUnit.SECONDS)); assertEquals(Integer.valueOf(12), result.toCompletableFuture().get(3, TimeUnit.SECONDS));
} }
@Test @Test
@ -588,7 +588,7 @@ public class GraphStageDocTest extends AbstractJavaTest {
.takeWithin(java.time.Duration.ofMillis(250)) .takeWithin(java.time.Duration.ofMillis(250))
.runFold(0, (n, sum) -> n + sum, system); .runFold(0, (n, sum) -> n + sum, system);
assertEquals(new Integer(1), result.toCompletableFuture().get(3, TimeUnit.SECONDS)); assertEquals(Integer.valueOf(1), result.toCompletableFuture().get(3, TimeUnit.SECONDS));
} }
// #materialized // #materialized
@ -654,12 +654,12 @@ public class GraphStageDocTest extends AbstractJavaTest {
// tests: // tests:
RunnableGraph<CompletionStage<Integer>> flow = RunnableGraph<CompletionStage<Integer>> flow =
Source.from(Arrays.asList(1, 2, 3)) Source.from(Arrays.asList(1, 2, 3))
.viaMat(new FirstValue(), Keep.right()) .viaMat(new FirstValue<Integer>(), Keep.right())
.to(Sink.ignore()); .to(Sink.ignore());
CompletionStage<Integer> result = flow.run(system); CompletionStage<Integer> result = flow.run(system);
assertEquals(new Integer(1), result.toCompletableFuture().get(3, TimeUnit.SECONDS)); assertEquals(Integer.valueOf(1), result.toCompletableFuture().get(3, TimeUnit.SECONDS));
} }
// #detached // #detached
@ -751,7 +751,7 @@ public class GraphStageDocTest extends AbstractJavaTest {
.via(new TwoBuffer<>()) .via(new TwoBuffer<>())
.runFold(0, (acc, n) -> acc + n, system); .runFold(0, (acc, n) -> acc + n, system);
assertEquals(new Integer(6), result1.toCompletableFuture().get(3, TimeUnit.SECONDS)); assertEquals(Integer.valueOf(6), result1.toCompletableFuture().get(3, TimeUnit.SECONDS));
TestSubscriber.ManualProbe<Integer> subscriber = TestSubscriber.manualProbe(system); TestSubscriber.ManualProbe<Integer> subscriber = TestSubscriber.manualProbe(system);
TestPublisher.Probe<Integer> publisher = TestPublisher.probe(0, system); TestPublisher.Probe<Integer> publisher = TestPublisher.probe(0, system);

View file

@ -4,6 +4,7 @@
package jdocs.stream; package jdocs.stream;
import akka.Done;
import akka.NotUsed; import akka.NotUsed;
import akka.actor.*; import akka.actor.*;
import akka.stream.*; import akka.stream.*;
@ -773,7 +774,15 @@ public class IntegrationDocTest extends AbstractJavaTest {
Source<Integer, ActorRef> source = Source<Integer, ActorRef> source =
Source.actorRef( Source.actorRef(
bufferSize, OverflowStrategy.dropHead()); // note: backpressure is not supported elem -> {
// complete stream immediately if we send it Done
if (elem == Done.done()) return Optional.of(CompletionStrategy.immediately());
else return Optional.empty();
},
// never fail the stream because of a message
elem -> Optional.empty(),
bufferSize,
OverflowStrategy.dropHead()); // note: backpressure is not supported
ActorRef actorRef = ActorRef actorRef =
source source
.map(x -> x * x) .map(x -> x * x)

View file

@ -69,7 +69,7 @@ public class RestartDocTest {
20, // limits the amount of restarts to 20 20, // limits the amount of restarts to 20
() -> () ->
// Create a source from a future of a source // Create a source from a future of a source
Source.fromSourceCompletionStage( Source.completionStageSource(
// Issue a GET request on the event stream // Issue a GET request on the event stream
Http.get(system) Http.get(system)
.singleRequest(HttpRequest.create("http://example.com/eventstream")) .singleRequest(HttpRequest.create("http://example.com/eventstream"))

View file

@ -54,8 +54,8 @@ public class StreamPartialGraphDSLDocTest extends AbstractJavaTest {
builder.from(zip1.out()).toInlet(zip2.in0()); builder.from(zip1.out()).toInlet(zip2.in0());
// return the shape, which has three inputs and one output // return the shape, which has three inputs and one output
return new UniformFanInShape<Integer, Integer>( return UniformFanInShape.<Integer, Integer>create(
zip2.out(), new Inlet[] {zip1.in0(), zip1.in1(), zip2.in1()}); zip2.out(), Arrays.asList(zip1.in0(), zip1.in1(), zip2.in1()));
}); });
final Sink<Integer, CompletionStage<Integer>> resultSink = Sink.<Integer>head(); final Sink<Integer, CompletionStage<Integer>> resultSink = Sink.<Integer>head();

View file

@ -8,11 +8,13 @@ import java.time.Duration;
import java.util.Arrays; import java.util.Arrays;
import java.util.Collections; import java.util.Collections;
import java.util.List; import java.util.List;
import java.util.Optional;
import java.util.concurrent.CompletableFuture; import java.util.concurrent.CompletableFuture;
import java.util.concurrent.CompletionStage; import java.util.concurrent.CompletionStage;
import java.util.concurrent.ExecutionException; import java.util.concurrent.ExecutionException;
import java.util.concurrent.TimeUnit; import java.util.concurrent.TimeUnit;
import akka.Done;
import akka.NotUsed; import akka.NotUsed;
import jdocs.AbstractJavaTest; import jdocs.AbstractJavaTest;
import akka.testkit.javadsl.TestKit; import akka.testkit.javadsl.TestKit;
@ -128,7 +130,16 @@ public class StreamTestKitDocTest extends AbstractJavaTest {
.toMat(Sink.fold("", (agg, next) -> agg + next), Keep.right()); .toMat(Sink.fold("", (agg, next) -> agg + next), Keep.right());
final Pair<ActorRef, CompletionStage<String>> refAndCompletionStage = final Pair<ActorRef, CompletionStage<String>> refAndCompletionStage =
Source.<Integer>actorRef(8, OverflowStrategy.fail()) Source.<Integer>actorRef(
elem -> {
// complete stream immediately if we send it Done
if (elem == Done.done()) return Optional.of(CompletionStrategy.immediately());
else return Optional.empty();
},
// never fail the stream because of a message
elem -> Optional.empty(),
8,
OverflowStrategy.fail())
.toMat(sinkUnderTest, Keep.both()) .toMat(sinkUnderTest, Keep.both())
.run(system); .run(system);
final ActorRef ref = refAndCompletionStage.first(); final ActorRef ref = refAndCompletionStage.first();
@ -137,7 +148,7 @@ public class StreamTestKitDocTest extends AbstractJavaTest {
ref.tell(1, ActorRef.noSender()); ref.tell(1, ActorRef.noSender());
ref.tell(2, ActorRef.noSender()); ref.tell(2, ActorRef.noSender());
ref.tell(3, ActorRef.noSender()); ref.tell(3, ActorRef.noSender());
ref.tell(new akka.actor.Status.Success("done"), ActorRef.noSender()); ref.tell(Done.getInstance(), ActorRef.noSender());
final String result = future.toCompletableFuture().get(1, TimeUnit.SECONDS); final String result = future.toCompletableFuture().get(1, TimeUnit.SECONDS);
assertEquals(result, "123"); assertEquals(result, "123");

View file

@ -5,6 +5,7 @@
package jdocs.stream.javadsl.cookbook; package jdocs.stream.javadsl.cookbook;
import akka.Done; import akka.Done;
import akka.NotUsed;
import akka.actor.ActorSystem; import akka.actor.ActorSystem;
import akka.dispatch.Futures; import akka.dispatch.Futures;
import akka.japi.pf.PFBuilder; import akka.japi.pf.PFBuilder;
@ -45,16 +46,18 @@ public class RecipeAdhocSourceTest extends RecipeTest {
// #adhoc-source // #adhoc-source
public <T> Source<T, ?> adhocSource(Source<T, ?> source, Duration timeout, int maxRetries) { public <T> Source<T, ?> adhocSource(Source<T, ?> source, Duration timeout, int maxRetries) {
return Source.lazily( return Source.lazySource(
() -> () ->
source source
.backpressureTimeout(timeout) .backpressureTimeout(timeout)
.recoverWithRetries( .recoverWithRetries(
maxRetries, maxRetries,
new PFBuilder() new PFBuilder<Throwable, Source<T, NotUsed>>()
.match( .match(
TimeoutException.class, TimeoutException.class,
ex -> Source.lazily(() -> source.backpressureTimeout(timeout))) ex ->
Source.lazySource(() -> source.backpressureTimeout(timeout))
.mapMaterializedValue(v -> NotUsed.getInstance()))
.build())); .build()));
} }
// #adhoc-source // #adhoc-source

View file

@ -30,7 +30,7 @@ public class StatefulMapConcat {
return (element) -> { return (element) -> {
counter[0] += 1; counter[0] += 1;
// we return an iterable with the single element // we return an iterable with the single element
return Arrays.asList(new Pair(element, counter[0])); return Arrays.asList(new Pair<String, Long>(element, counter[0]));
}; };
}); });

View file

@ -159,8 +159,8 @@ public class PersistentActorCompileOnlyTest {
} }
@Override @Override
public SnapshotSelectionCriteria snapshotSelectionCriteria() { public Recovery recovery() {
return SnapshotSelectionCriteria.none(); return Recovery.withSnapshotSelectionCriteria(SnapshotSelectionCriteria.none());
} }
@Override @Override

View file

@ -539,9 +539,11 @@ public class PersistentActorJavaDslTest extends JUnitSuite {
.runWith(Sink.seq(), testKit.system()) .runWith(Sink.seq(), testKit.system())
.toCompletableFuture() .toCompletableFuture()
.get(); .get();
assertEquals( assertEquals(1, events.size());
Lists.newArrayList(new EventEnvelope(new Sequence(1), "tagging", 1, new Incremented(1))), EventEnvelope eventEnvelope = events.get(0);
events); assertEquals(new Sequence(1), eventEnvelope.offset());
assertEquals("tagging", eventEnvelope.persistenceId());
assertEquals(new Incremented(1), eventEnvelope.event());
} }
@Test @Test
@ -570,10 +572,11 @@ public class PersistentActorJavaDslTest extends JUnitSuite {
.runWith(Sink.seq(), testKit.system()) .runWith(Sink.seq(), testKit.system())
.toCompletableFuture() .toCompletableFuture()
.get(); .get();
assertEquals( assertEquals(1, events.size());
Lists.newArrayList( EventEnvelope eventEnvelope = events.get(0);
new EventEnvelope(new Sequence(1), "transform", 1, new Wrapper(new Incremented(1)))), assertEquals(new Sequence(1), eventEnvelope.offset());
events); assertEquals("transform", eventEnvelope.persistenceId());
assertEquals(new Wrapper(new Incremented(1)), eventEnvelope.event());
ActorRef<Command> c2 = testKit.spawn(transformer); ActorRef<Command> c2 = testKit.spawn(transformer);
c2.tell(new GetValue(stateProbe.ref())); c2.tell(new GetValue(stateProbe.ref()));

View file

@ -11,8 +11,9 @@ import akka.persistence.typed.javadsl.*;
import java.time.Duration; import java.time.Duration;
import static akka.persistence.fsm.AbstractPersistentFSMTest.WebStoreCustomerFSM.*; import static jdocs.akka.persistence.typed.WebStoreCustomerFSM.*;
@Deprecated
public class PersistentFsmToTypedMigrationCompileOnlyTest { public class PersistentFsmToTypedMigrationCompileOnlyTest {
// #commands // #commands
@ -97,7 +98,7 @@ public class PersistentFsmToTypedMigrationCompileOnlyTest {
@Override @Override
public EventSeq<DomainEvent> fromJournal(Object event, String manifest) { public EventSeq<DomainEvent> fromJournal(Object event, String manifest) {
if (event instanceof StateChangeEvent) { if (event instanceof akka.persistence.fsm.PersistentFSM.StateChangeEvent) {
// In this example the state transitions can be inferred from the events // In this example the state transitions can be inferred from the events
// Alternatively the StateChangeEvent can be converted to a private event if either the // Alternatively the StateChangeEvent can be converted to a private event if either the
// StateChangeEvent.stateIdentifier // StateChangeEvent.stateIdentifier

View file

@ -0,0 +1,150 @@
/*
* Copyright (C) 2019-2020 Lightbend Inc. <https://www.lightbend.com>
*/
package jdocs.akka.persistence.typed;
import java.io.Serializable;
import java.util.Collections;
import java.util.List;
import java.util.ArrayList;
/**
* The model from akka.persistence.fsm.AbstractPersistentFSMTest.WebStoreCustomerFSM brought here
* for the PersistentFsmToTypedMigrationCompileOnlyTest
*/
public class WebStoreCustomerFSM {
public static class ShoppingCart {
private final List<Item> items = new ArrayList<>();
public ShoppingCart(Item initialItem) {
items.add(initialItem);
}
public ShoppingCart() {}
public List<Item> getItems() {
return Collections.unmodifiableList(items);
}
public ShoppingCart addItem(Item item) {
items.add(item);
return this;
}
public void empty() {
items.clear();
}
}
public static class Item implements Serializable {
private final String id;
private final String name;
private final float price;
Item(String id, String name, float price) {
this.id = id;
this.name = name;
this.price = price;
}
public String getId() {
return id;
}
public float getPrice() {
return price;
}
public String getName() {
return name;
}
@Override
public String toString() {
return String.format("Item{id=%s, name=%s, price=%s}", id, price, name);
}
@Override
public boolean equals(Object o) {
if (this == o) return true;
if (o == null || getClass() != o.getClass()) return false;
Item item = (Item) o;
return item.price == price && id.equals(item.id) && name.equals(item.name);
}
}
public interface Command {}
public static final class AddItem implements Command {
private final Item item;
public AddItem(Item item) {
this.item = item;
}
public Item getItem() {
return item;
}
}
public enum Buy implements Command {
INSTANCE
}
public enum Leave implements Command {
INSTANCE
}
public enum GetCurrentCart implements Command {
INSTANCE
}
public interface DomainEvent extends Serializable {}
public static final class ItemAdded implements DomainEvent {
private final Item item;
public ItemAdded(Item item) {
this.item = item;
}
public Item getItem() {
return item;
}
}
public enum OrderExecuted implements DomainEvent {
INSTANCE
}
public enum OrderDiscarded implements DomainEvent {
INSTANCE
}
public enum CustomerInactive implements DomainEvent {
INSTANCE
}
// Side effects - report events to be sent to some "Report Actor"
public interface ReportEvent {}
public static final class PurchaseWasMade implements ReportEvent {
private final List<Item> items;
public PurchaseWasMade(List<Item> items) {
this.items = Collections.unmodifiableList(items);
}
public List<Item> getItems() {
return items;
}
}
public enum ShoppingCardDiscarded implements ReportEvent {
INSTANCE
}
}

View file

@ -4,8 +4,6 @@
package akka.persistence.fsm.japi.pf; package akka.persistence.fsm.japi.pf;
import akka.persistence.fsm.PersistentFSM;
import akka.persistence.fsm.PersistentFSMBase;
import akka.japi.pf.FI; import akka.japi.pf.FI;
import akka.japi.pf.PFBuilder; import akka.japi.pf.PFBuilder;
import scala.PartialFunction; import scala.PartialFunction;
@ -15,15 +13,22 @@ import java.util.List;
/** /**
* Builder used to create a partial function for {@link akka.actor.FSM#whenUnhandled}. * Builder used to create a partial function for {@link akka.actor.FSM#whenUnhandled}.
* *
* @deprecated use EventSourcedBehavior since 2.6.0
* @param <S> the state type * @param <S> the state type
* @param <D> the data type * @param <D> the data type
* @param <E> the domain event type * @param <E> the domain event type
*/ */
@SuppressWarnings("rawtypes") @SuppressWarnings("rawtypes")
@Deprecated
public class FSMStateFunctionBuilder<S, D, E> { public class FSMStateFunctionBuilder<S, D, E> {
private PFBuilder<PersistentFSM.Event<D>, PersistentFSM.State<S, D, E>> builder = private PFBuilder<
new PFBuilder<PersistentFSM.Event<D>, PersistentFSM.State<S, D, E>>(); akka.persistence.fsm.PersistentFSM.Event<D>,
akka.persistence.fsm.PersistentFSM.State<S, D, E>>
builder =
new PFBuilder<
akka.persistence.fsm.PersistentFSM.Event<D>,
akka.persistence.fsm.PersistentFSM.State<S, D, E>>();
/** /**
* An erased processing of the event matcher. The compile time checks are enforced by the public * An erased processing of the event matcher. The compile time checks are enforced by the public
@ -47,10 +52,10 @@ public class FSMStateFunctionBuilder<S, D, E> {
final FI.TypedPredicate2 predicate, final FI.TypedPredicate2 predicate,
final FI.Apply2 apply) { final FI.Apply2 apply) {
builder.match( builder.match(
PersistentFSM.Event.class, akka.persistence.fsm.PersistentFSM.Event.class,
new FI.TypedPredicate<PersistentFSM.Event>() { new FI.TypedPredicate<akka.persistence.fsm.PersistentFSM.Event>() {
@Override @Override
public boolean defined(PersistentFSM.Event e) { public boolean defined(akka.persistence.fsm.PersistentFSM.Event e) {
boolean res = true; boolean res = true;
if (eventOrType != null) { if (eventOrType != null) {
if (eventOrType instanceof Class) { if (eventOrType instanceof Class) {
@ -76,11 +81,15 @@ public class FSMStateFunctionBuilder<S, D, E> {
return res; return res;
} }
}, },
new FI.Apply<PersistentFSM.Event, PersistentFSM.State<S, D, E>>() { new FI.Apply<
public PersistentFSM.State<S, D, E> apply(PersistentFSM.Event e) throws Exception { akka.persistence.fsm.PersistentFSM.Event,
akka.persistence.fsm.PersistentFSM.State<S, D, E>>() {
public akka.persistence.fsm.PersistentFSM.State<S, D, E> apply(
akka.persistence.fsm.PersistentFSM.Event e) throws Exception {
@SuppressWarnings("unchecked") @SuppressWarnings("unchecked")
PersistentFSM.State<S, D, E> res = akka.persistence.fsm.PersistentFSM.State<S, D, E> res =
(PersistentFSM.State<S, D, E>) apply.apply(e.event(), e.stateData()); (akka.persistence.fsm.PersistentFSM.State<S, D, E>)
apply.apply(e.event(), e.stateData());
return res; return res;
} }
}); });
@ -103,7 +112,7 @@ public class FSMStateFunctionBuilder<S, D, E> {
final Class<P> eventType, final Class<P> eventType,
final Class<Q> dataType, final Class<Q> dataType,
final FI.TypedPredicate2<P, Q> predicate, final FI.TypedPredicate2<P, Q> predicate,
final FI.Apply2<P, Q, PersistentFSM.State<S, D, E>> apply) { final FI.Apply2<P, Q, akka.persistence.fsm.PersistentFSM.State<S, D, E>> apply) {
erasedEvent(eventType, dataType, predicate, apply); erasedEvent(eventType, dataType, predicate, apply);
return this; return this;
} }
@ -121,7 +130,7 @@ public class FSMStateFunctionBuilder<S, D, E> {
public <P, Q> FSMStateFunctionBuilder<S, D, E> event( public <P, Q> FSMStateFunctionBuilder<S, D, E> event(
final Class<P> eventType, final Class<P> eventType,
final Class<Q> dataType, final Class<Q> dataType,
final FI.Apply2<P, Q, PersistentFSM.State<S, D, E>> apply) { final FI.Apply2<P, Q, akka.persistence.fsm.PersistentFSM.State<S, D, E>> apply) {
return erasedEvent(eventType, dataType, null, apply); return erasedEvent(eventType, dataType, null, apply);
} }
@ -136,7 +145,7 @@ public class FSMStateFunctionBuilder<S, D, E> {
public <P> FSMStateFunctionBuilder<S, D, E> event( public <P> FSMStateFunctionBuilder<S, D, E> event(
final Class<P> eventType, final Class<P> eventType,
final FI.TypedPredicate2<P, D> predicate, final FI.TypedPredicate2<P, D> predicate,
final FI.Apply2<P, D, PersistentFSM.State<S, D, E>> apply) { final FI.Apply2<P, D, akka.persistence.fsm.PersistentFSM.State<S, D, E>> apply) {
return erasedEvent(eventType, null, predicate, apply); return erasedEvent(eventType, null, predicate, apply);
} }
@ -148,7 +157,8 @@ public class FSMStateFunctionBuilder<S, D, E> {
* @return the builder with the case statement added * @return the builder with the case statement added
*/ */
public <P> FSMStateFunctionBuilder<S, D, E> event( public <P> FSMStateFunctionBuilder<S, D, E> event(
final Class<P> eventType, final FI.Apply2<P, D, PersistentFSM.State<S, D, E>> apply) { final Class<P> eventType,
final FI.Apply2<P, D, akka.persistence.fsm.PersistentFSM.State<S, D, E>> apply) {
return erasedEvent(eventType, null, null, apply); return erasedEvent(eventType, null, null, apply);
} }
@ -161,7 +171,7 @@ public class FSMStateFunctionBuilder<S, D, E> {
*/ */
public FSMStateFunctionBuilder<S, D, E> event( public FSMStateFunctionBuilder<S, D, E> event(
final FI.TypedPredicate2<Object, D> predicate, final FI.TypedPredicate2<Object, D> predicate,
final FI.Apply2<Object, D, PersistentFSM.State<S, D, E>> apply) { final FI.Apply2<Object, D, akka.persistence.fsm.PersistentFSM.State<S, D, E>> apply) {
return erasedEvent(null, null, predicate, apply); return erasedEvent(null, null, predicate, apply);
} }
@ -178,12 +188,12 @@ public class FSMStateFunctionBuilder<S, D, E> {
public <Q> FSMStateFunctionBuilder<S, D, E> event( public <Q> FSMStateFunctionBuilder<S, D, E> event(
final List<Object> eventMatches, final List<Object> eventMatches,
final Class<Q> dataType, final Class<Q> dataType,
final FI.Apply2<Object, Q, PersistentFSM.State<S, D, E>> apply) { final FI.Apply2<Object, Q, akka.persistence.fsm.PersistentFSM.State<S, D, E>> apply) {
builder.match( builder.match(
PersistentFSM.Event.class, akka.persistence.fsm.PersistentFSM.Event.class,
new FI.TypedPredicate<PersistentFSM.Event>() { new FI.TypedPredicate<akka.persistence.fsm.PersistentFSM.Event>() {
@Override @Override
public boolean defined(PersistentFSM.Event e) { public boolean defined(akka.persistence.fsm.PersistentFSM.Event e) {
if (dataType != null && !dataType.isInstance(e.stateData())) return false; if (dataType != null && !dataType.isInstance(e.stateData())) return false;
boolean emMatch = false; boolean emMatch = false;
@ -200,8 +210,11 @@ public class FSMStateFunctionBuilder<S, D, E> {
return emMatch; return emMatch;
} }
}, },
new FI.Apply<PersistentFSM.Event, PersistentFSM.State<S, D, E>>() { new FI.Apply<
public PersistentFSM.State<S, D, E> apply(PersistentFSM.Event e) throws Exception { akka.persistence.fsm.PersistentFSM.Event,
akka.persistence.fsm.PersistentFSM.State<S, D, E>>() {
public akka.persistence.fsm.PersistentFSM.State<S, D, E> apply(
akka.persistence.fsm.PersistentFSM.Event e) throws Exception {
@SuppressWarnings("unchecked") @SuppressWarnings("unchecked")
Q q = (Q) e.stateData(); Q q = (Q) e.stateData();
return apply.apply(e.event(), q); return apply.apply(e.event(), q);
@ -221,7 +234,7 @@ public class FSMStateFunctionBuilder<S, D, E> {
*/ */
public FSMStateFunctionBuilder<S, D, E> event( public FSMStateFunctionBuilder<S, D, E> event(
final List<Object> eventMatches, final List<Object> eventMatches,
final FI.Apply2<Object, D, PersistentFSM.State<S, D, E>> apply) { final FI.Apply2<Object, D, akka.persistence.fsm.PersistentFSM.State<S, D, E>> apply) {
return event(eventMatches, null, apply); return event(eventMatches, null, apply);
} }
@ -237,7 +250,7 @@ public class FSMStateFunctionBuilder<S, D, E> {
public <P, Q> FSMStateFunctionBuilder<S, D, E> eventEquals( public <P, Q> FSMStateFunctionBuilder<S, D, E> eventEquals(
final P event, final P event,
final Class<Q> dataType, final Class<Q> dataType,
final FI.Apply2<P, Q, PersistentFSM.State<S, D, E>> apply) { final FI.Apply2<P, Q, akka.persistence.fsm.PersistentFSM.State<S, D, E>> apply) {
return erasedEvent(event, dataType, null, apply); return erasedEvent(event, dataType, null, apply);
} }
@ -249,7 +262,8 @@ public class FSMStateFunctionBuilder<S, D, E> {
* @return the builder with the case statement added * @return the builder with the case statement added
*/ */
public <P> FSMStateFunctionBuilder<S, D, E> eventEquals( public <P> FSMStateFunctionBuilder<S, D, E> eventEquals(
final P event, final FI.Apply2<P, D, PersistentFSM.State<S, D, E>> apply) { final P event,
final FI.Apply2<P, D, akka.persistence.fsm.PersistentFSM.State<S, D, E>> apply) {
return erasedEvent(event, null, null, apply); return erasedEvent(event, null, null, apply);
} }
@ -260,7 +274,7 @@ public class FSMStateFunctionBuilder<S, D, E> {
* @return the builder with the case statement added * @return the builder with the case statement added
*/ */
public FSMStateFunctionBuilder<S, D, E> anyEvent( public FSMStateFunctionBuilder<S, D, E> anyEvent(
final FI.Apply2<Object, D, PersistentFSM.State<S, D, E>> apply) { final FI.Apply2<Object, D, akka.persistence.fsm.PersistentFSM.State<S, D, E>> apply) {
return erasedEvent(null, null, null, apply); return erasedEvent(null, null, null, apply);
} }
@ -270,7 +284,10 @@ public class FSMStateFunctionBuilder<S, D, E> {
* *
* @return a PartialFunction for this builder. * @return a PartialFunction for this builder.
*/ */
public PartialFunction<PersistentFSM.Event<D>, PersistentFSM.State<S, D, E>> build() { public PartialFunction<
akka.persistence.fsm.PersistentFSM.Event<D>,
akka.persistence.fsm.PersistentFSM.State<S, D, E>>
build() {
return builder.build(); return builder.build();
} }
} }

View file

@ -4,8 +4,6 @@
package akka.persistence.fsm.japi.pf; package akka.persistence.fsm.japi.pf;
import akka.persistence.fsm.PersistentFSM;
import akka.persistence.fsm.PersistentFSMBase;
import akka.japi.pf.FI; import akka.japi.pf.FI;
import akka.japi.pf.UnitPFBuilder; import akka.japi.pf.UnitPFBuilder;
import scala.PartialFunction; import scala.PartialFunction;
@ -14,12 +12,15 @@ import scala.runtime.BoxedUnit;
/** /**
* Builder used to create a partial function for {@link akka.actor.FSM#onTermination}. * Builder used to create a partial function for {@link akka.actor.FSM#onTermination}.
* *
* @deprecated use EventSourcedBehavior since 2.6.0
* @param <S> the state type * @param <S> the state type
* @param <D> the data type * @param <D> the data type
*/ */
@Deprecated
public class FSMStopBuilder<S, D> { public class FSMStopBuilder<S, D> {
private UnitPFBuilder<PersistentFSM.StopEvent<S, D>> builder = new UnitPFBuilder<>(); private UnitPFBuilder<akka.persistence.fsm.PersistentFSM.StopEvent<S, D>> builder =
new UnitPFBuilder<>();
/** /**
* Add a case statement that matches on an {@link akka.actor.FSM.Reason}. * Add a case statement that matches on an {@link akka.actor.FSM.Reason}.
@ -29,17 +30,17 @@ public class FSMStopBuilder<S, D> {
* @return the builder with the case statement added * @return the builder with the case statement added
*/ */
public FSMStopBuilder<S, D> stop( public FSMStopBuilder<S, D> stop(
final PersistentFSM.Reason reason, final FI.UnitApply2<S, D> apply) { final akka.persistence.fsm.PersistentFSM.Reason reason, final FI.UnitApply2<S, D> apply) {
builder.match( builder.match(
PersistentFSM.StopEvent.class, akka.persistence.fsm.PersistentFSM.StopEvent.class,
new FI.TypedPredicate<PersistentFSM.StopEvent>() { new FI.TypedPredicate<akka.persistence.fsm.PersistentFSM.StopEvent>() {
@Override @Override
public boolean defined(PersistentFSM.StopEvent e) { public boolean defined(akka.persistence.fsm.PersistentFSM.StopEvent e) {
return reason.equals(e.reason()); return reason.equals(e.reason());
} }
}, },
new FI.UnitApply<PersistentFSM.StopEvent>() { new FI.UnitApply<akka.persistence.fsm.PersistentFSM.StopEvent>() {
public void apply(PersistentFSM.StopEvent e) throws Exception { public void apply(akka.persistence.fsm.PersistentFSM.StopEvent e) throws Exception {
@SuppressWarnings("unchecked") @SuppressWarnings("unchecked")
S s = (S) e.currentState(); S s = (S) e.currentState();
@SuppressWarnings("unchecked") @SuppressWarnings("unchecked")
@ -59,7 +60,7 @@ public class FSMStopBuilder<S, D> {
* @param <P> the reason type to match on * @param <P> the reason type to match on
* @return the builder with the case statement added * @return the builder with the case statement added
*/ */
public <P extends PersistentFSM.Reason> FSMStopBuilder<S, D> stop( public <P extends akka.persistence.fsm.PersistentFSM.Reason> FSMStopBuilder<S, D> stop(
final Class<P> reasonType, final FI.UnitApply3<P, S, D> apply) { final Class<P> reasonType, final FI.UnitApply3<P, S, D> apply) {
return this.stop( return this.stop(
reasonType, reasonType,
@ -81,15 +82,15 @@ public class FSMStopBuilder<S, D> {
* @param <P> the reason type to match on * @param <P> the reason type to match on
* @return the builder with the case statement added * @return the builder with the case statement added
*/ */
public <P extends PersistentFSM.Reason> FSMStopBuilder<S, D> stop( public <P extends akka.persistence.fsm.PersistentFSM.Reason> FSMStopBuilder<S, D> stop(
final Class<P> reasonType, final Class<P> reasonType,
final FI.TypedPredicate<P> predicate, final FI.TypedPredicate<P> predicate,
final FI.UnitApply3<P, S, D> apply) { final FI.UnitApply3<P, S, D> apply) {
builder.match( builder.match(
PersistentFSM.StopEvent.class, akka.persistence.fsm.PersistentFSM.StopEvent.class,
new FI.TypedPredicate<PersistentFSM.StopEvent>() { new FI.TypedPredicate<akka.persistence.fsm.PersistentFSM.StopEvent>() {
@Override @Override
public boolean defined(PersistentFSM.StopEvent e) { public boolean defined(akka.persistence.fsm.PersistentFSM.StopEvent e) {
if (reasonType.isInstance(e.reason())) { if (reasonType.isInstance(e.reason())) {
@SuppressWarnings("unchecked") @SuppressWarnings("unchecked")
P p = (P) e.reason(); P p = (P) e.reason();
@ -99,8 +100,8 @@ public class FSMStopBuilder<S, D> {
} }
} }
}, },
new FI.UnitApply<PersistentFSM.StopEvent>() { new FI.UnitApply<akka.persistence.fsm.PersistentFSM.StopEvent>() {
public void apply(PersistentFSM.StopEvent e) throws Exception { public void apply(akka.persistence.fsm.PersistentFSM.StopEvent e) throws Exception {
@SuppressWarnings("unchecked") @SuppressWarnings("unchecked")
P p = (P) e.reason(); P p = (P) e.reason();
@SuppressWarnings("unchecked") @SuppressWarnings("unchecked")
@ -120,7 +121,7 @@ public class FSMStopBuilder<S, D> {
* *
* @return a PartialFunction for this builder. * @return a PartialFunction for this builder.
*/ */
public PartialFunction<PersistentFSM.StopEvent<S, D>, BoxedUnit> build() { public PartialFunction<akka.persistence.fsm.PersistentFSM.StopEvent<S, D>, BoxedUnit> build() {
return builder.build(); return builder.build();
} }
} }

View file

@ -19,12 +19,9 @@ import java.util.List;
import java.util.UUID; import java.util.UUID;
import java.time.Duration; import java.time.Duration;
import akka.persistence.fsm.PersistentFSM.CurrentState;
import org.junit.Test; import org.junit.Test;
import org.scalatestplus.junit.JUnitSuite; import org.scalatestplus.junit.JUnitSuite;
import static akka.persistence.fsm.PersistentFSM.FSMState;
import static akka.persistence.fsm.AbstractPersistentFSMTest.WebStoreCustomerFSM.UserState; import static akka.persistence.fsm.AbstractPersistentFSMTest.WebStoreCustomerFSM.UserState;
import static akka.persistence.fsm.AbstractPersistentFSMTest.WebStoreCustomerFSM.ShoppingCart; import static akka.persistence.fsm.AbstractPersistentFSMTest.WebStoreCustomerFSM.ShoppingCart;
import static akka.persistence.fsm.AbstractPersistentFSMTest.WebStoreCustomerFSM.Item; import static akka.persistence.fsm.AbstractPersistentFSMTest.WebStoreCustomerFSM.Item;
@ -43,6 +40,7 @@ import static org.junit.Assert.assertEquals;
import static org.junit.Assert.assertThat; import static org.junit.Assert.assertThat;
import static org.junit.Assert.assertTrue; import static org.junit.Assert.assertTrue;
@Deprecated
public class AbstractPersistentFSMTest extends JUnitSuite { public class AbstractPersistentFSMTest extends JUnitSuite {
private static Option<String> none = Option.none(); private static Option<String> none = Option.none();
@ -83,7 +81,7 @@ public class AbstractPersistentFSMTest extends JUnitSuite {
fsmRef.tell(GetCurrentCart.INSTANCE, getRef()); fsmRef.tell(GetCurrentCart.INSTANCE, getRef());
fsmRef.tell(Leave.INSTANCE, getRef()); fsmRef.tell(Leave.INSTANCE, getRef());
CurrentState currentState = PersistentFSM.CurrentState currentState =
expectMsgClass(akka.persistence.fsm.PersistentFSM.CurrentState.class); expectMsgClass(akka.persistence.fsm.PersistentFSM.CurrentState.class);
assertEquals(currentState.state(), UserState.LOOKING_AROUND); assertEquals(currentState.state(), UserState.LOOKING_AROUND);
@ -129,7 +127,7 @@ public class AbstractPersistentFSMTest extends JUnitSuite {
fsmRef.tell(new AddItem(shirt), getRef()); fsmRef.tell(new AddItem(shirt), getRef());
CurrentState currentState = PersistentFSM.CurrentState currentState =
expectMsgClass(akka.persistence.fsm.PersistentFSM.CurrentState.class); expectMsgClass(akka.persistence.fsm.PersistentFSM.CurrentState.class);
assertEquals(currentState.state(), UserState.LOOKING_AROUND); assertEquals(currentState.state(), UserState.LOOKING_AROUND);
@ -171,7 +169,7 @@ public class AbstractPersistentFSMTest extends JUnitSuite {
fsmRef.tell(new AddItem(shoes), getRef()); fsmRef.tell(new AddItem(shoes), getRef());
fsmRef.tell(GetCurrentCart.INSTANCE, getRef()); fsmRef.tell(GetCurrentCart.INSTANCE, getRef());
CurrentState currentState = PersistentFSM.CurrentState currentState =
expectMsgClass(akka.persistence.fsm.PersistentFSM.CurrentState.class); expectMsgClass(akka.persistence.fsm.PersistentFSM.CurrentState.class);
assertEquals(currentState.state(), UserState.LOOKING_AROUND); assertEquals(currentState.state(), UserState.LOOKING_AROUND);
@ -247,7 +245,7 @@ public class AbstractPersistentFSMTest extends JUnitSuite {
fsmRef.tell(Buy.INSTANCE, getRef()); fsmRef.tell(Buy.INSTANCE, getRef());
fsmRef.tell(Leave.INSTANCE, getRef()); fsmRef.tell(Leave.INSTANCE, getRef());
CurrentState currentState = PersistentFSM.CurrentState currentState =
expectMsgClass(akka.persistence.fsm.PersistentFSM.CurrentState.class); expectMsgClass(akka.persistence.fsm.PersistentFSM.CurrentState.class);
assertEquals(currentState.state(), UserState.LOOKING_AROUND); assertEquals(currentState.state(), UserState.LOOKING_AROUND);
@ -287,7 +285,7 @@ public class AbstractPersistentFSMTest extends JUnitSuite {
fsmRef.tell(new AddItem(coat), getRef()); fsmRef.tell(new AddItem(coat), getRef());
fsmRef.tell(Leave.INSTANCE, getRef()); fsmRef.tell(Leave.INSTANCE, getRef());
CurrentState currentState = PersistentFSM.CurrentState currentState =
expectMsgClass(akka.persistence.fsm.PersistentFSM.CurrentState.class); expectMsgClass(akka.persistence.fsm.PersistentFSM.CurrentState.class);
assertEquals(currentState.state(), UserState.LOOKING_AROUND); assertEquals(currentState.state(), UserState.LOOKING_AROUND);
@ -316,7 +314,7 @@ public class AbstractPersistentFSMTest extends JUnitSuite {
fsmRef.tell(new AddItem(shirt), getRef()); fsmRef.tell(new AddItem(shirt), getRef());
CurrentState currentState = PersistentFSM.CurrentState currentState =
expectMsgClass(akka.persistence.fsm.PersistentFSM.CurrentState.class); expectMsgClass(akka.persistence.fsm.PersistentFSM.CurrentState.class);
assertEquals(currentState.state(), UserState.LOOKING_AROUND); assertEquals(currentState.state(), UserState.LOOKING_AROUND);
@ -665,7 +663,7 @@ public class AbstractPersistentFSMTest extends JUnitSuite {
// #customer-apply-event // #customer-apply-event
} }
enum PFSMState implements FSMState { enum PFSMState implements PersistentFSM.FSMState {
STARTED; STARTED;
@Override @Override

View file

@ -29,19 +29,19 @@ public class JavaFlowSupportCompileTest extends JUnitSuite {
}; };
final Source<String, Flow.Subscriber<String>> stringSubscriberSource = final Source<String, Flow.Subscriber<String>> stringSubscriberSource =
JavaFlowSupport.Source.asSubscriber(); JavaFlowSupport.Source.asSubscriber();
final Source<String, NotUsed> stringNotUsedSource = final Source<String, NotUsed> stringNotUsedSource =
JavaFlowSupport.Source.fromPublisher(processor); JavaFlowSupport.Source.fromPublisher(processor);
final akka.stream.javadsl.Flow<String, String, NotUsed> stringStringNotUsedFlow = final akka.stream.javadsl.Flow<String, String, NotUsed> stringStringNotUsedFlow =
JavaFlowSupport.Flow.fromProcessor(() -> processor); JavaFlowSupport.Flow.fromProcessor(() -> processor);
final akka.stream.javadsl.Flow<String, String, NotUsed> stringStringNotUsedFlow1 = final akka.stream.javadsl.Flow<String, String, NotUsed> stringStringNotUsedFlow1 =
JavaFlowSupport.Flow.fromProcessorMat(() -> Pair.apply(processor, NotUsed.getInstance())); JavaFlowSupport.Flow.fromProcessorMat(() -> Pair.apply(processor, NotUsed.getInstance()));
final Sink<String, Flow.Publisher<String>> stringPublisherSink = final Sink<String, Flow.Publisher<String>> stringPublisherSink =
JavaFlowSupport.Sink.asPublisher(AsPublisher.WITH_FANOUT); JavaFlowSupport.Sink.asPublisher(AsPublisher.WITH_FANOUT);
final Sink<String, NotUsed> stringNotUsedSink = final Sink<String, NotUsed> stringNotUsedSink =
JavaFlowSupport.Sink.fromSubscriber(processor); JavaFlowSupport.Sink.fromSubscriber(processor);
} }
} }

View file

@ -16,9 +16,9 @@ import akka.util.ByteString;
import org.junit.ClassRule; import org.junit.ClassRule;
import org.junit.Test; import org.junit.Test;
import scala.concurrent.Future; import scala.concurrent.Future;
import scala.concurrent.duration.FiniteDuration;
import java.io.InputStream; import java.io.InputStream;
import java.time.Duration;
import java.util.Arrays; import java.util.Arrays;
import java.util.Collections; import java.util.Collections;
import java.util.List; import java.util.List;
@ -37,7 +37,7 @@ public class InputStreamSinkTest extends StreamTest {
@Test @Test
public void mustReadEventViaInputStream() throws Exception { public void mustReadEventViaInputStream() throws Exception {
final FiniteDuration timeout = FiniteDuration.create(300, TimeUnit.MILLISECONDS); final Duration timeout = Duration.ofMillis(300);
final Sink<ByteString, InputStream> sink = StreamConverters.asInputStream(timeout); final Sink<ByteString, InputStream> sink = StreamConverters.asInputStream(timeout);
final List<ByteString> list = Collections.singletonList(ByteString.fromString("a")); final List<ByteString> list = Collections.singletonList(ByteString.fromString("a"));

View file

@ -59,12 +59,14 @@ public class AttributesTest extends StreamTest {
Optional.of(new Attributes.Name("b")), attributes.getAttribute(Attributes.Name.class)); Optional.of(new Attributes.Name("b")), attributes.getAttribute(Attributes.Name.class));
} }
@Deprecated
@Test @Test
public void mustGetPossiblyMissingFirstAttributeByClass() { public void mustGetPossiblyMissingFirstAttributeByClass() {
assertEquals( assertEquals(
Optional.of(new Attributes.Name("a")), attributes.getFirstAttribute(Attributes.Name.class)); Optional.of(new Attributes.Name("a")), attributes.getFirstAttribute(Attributes.Name.class));
} }
@Deprecated
@Test @Test
public void mustGetMissingFirstAttributeByClass() { public void mustGetMissingFirstAttributeByClass() {
assertEquals(Optional.empty(), attributes.getFirstAttribute(Attributes.LogLevels.class)); assertEquals(Optional.empty(), attributes.getFirstAttribute(Attributes.LogLevels.class));

View file

@ -491,7 +491,8 @@ public class FlowTest extends StreamTest {
final CompletionStage<Graph<SourceShape<String>, NotUsed>> stage = final CompletionStage<Graph<SourceShape<String>, NotUsed>> stage =
CompletableFuture.supplyAsync(fn); CompletableFuture.supplyAsync(fn);
final Source<String, CompletionStage<NotUsed>> source = Source.fromSourceCompletionStage(stage); final Source<String, CompletionStage<NotUsed>> source =
Source.completionStageSource(stage.thenApply(Source::fromGraph));
// collecting // collecting
final Publisher<String> pub = source.runWith(publisher, system); final Publisher<String> pub = source.runWith(publisher, system);
@ -881,11 +882,13 @@ public class FlowTest extends StreamTest {
if (elem == 2) throw new RuntimeException("ex"); if (elem == 2) throw new RuntimeException("ex");
else return elem; else return elem;
}) })
.recover( .recoverWithRetries(
new JavaPartialFunction<Throwable, Integer>() { 1,
public Integer apply(Throwable elem, boolean isCheck) { new JavaPartialFunction<Throwable, Graph<SourceShape<Integer>, NotUsed>>() {
if (isCheck) return null; public Graph<SourceShape<Integer>, NotUsed> apply(
return 0; Throwable elem, boolean isCheck) {
if (isCheck) return Source.empty();
return Source.single(0);
} }
}); });
@ -919,7 +922,7 @@ public class FlowTest extends StreamTest {
if (elem == 2) throw new RuntimeException("ex"); if (elem == 2) throw new RuntimeException("ex");
else return elem; else return elem;
}) })
.recover(RuntimeException.class, () -> 0); .recoverWithRetries(1, RuntimeException.class, () -> Source.single(0));
final CompletionStage<Done> future = final CompletionStage<Done> future =
source source
@ -983,6 +986,7 @@ public class FlowTest extends StreamTest {
TestPublisher.manualProbe(true, system); TestPublisher.manualProbe(true, system);
final TestKit probe = new TestKit(system); final TestKit probe = new TestKit(system);
final Iterable<Integer> recover = Arrays.asList(55, 0); final Iterable<Integer> recover = Arrays.asList(55, 0);
final int maxRetries = 10;
final Source<Integer, NotUsed> source = Source.fromPublisher(publisherProbe); final Source<Integer, NotUsed> source = Source.fromPublisher(publisherProbe);
final Flow<Integer, Integer, NotUsed> flow = final Flow<Integer, Integer, NotUsed> flow =
@ -992,7 +996,7 @@ public class FlowTest extends StreamTest {
if (elem == 2) throw new RuntimeException("ex"); if (elem == 2) throw new RuntimeException("ex");
else return elem; else return elem;
}) })
.recoverWith(RuntimeException.class, () -> Source.from(recover)); .recoverWithRetries(maxRetries, RuntimeException.class, () -> Source.from(recover));
final CompletionStage<Done> future = final CompletionStage<Done> future =
source source
@ -1028,7 +1032,9 @@ public class FlowTest extends StreamTest {
}) })
.recoverWithRetries( .recoverWithRetries(
3, 3,
new PFBuilder().match(RuntimeException.class, ex -> Source.from(recover)).build()); new PFBuilder<Throwable, Graph<SourceShape<Integer>, NotUsed>>()
.match(RuntimeException.class, ex -> Source.from(recover))
.build());
final CompletionStage<Done> future = final CompletionStage<Done> future =
source source
@ -1191,7 +1197,12 @@ public class FlowTest extends StreamTest {
})); }));
final TestKit probe = new TestKit(system); final TestKit probe = new TestKit(system);
Source<String, ActorRef> source = Source.actorRef(1, OverflowStrategy.dropNew()); Source<String, ActorRef> source =
Source.actorRef(
msg -> Optional.<CompletionStrategy>empty(),
msg -> Optional.<Throwable>empty(),
1,
OverflowStrategy.dropNew());
final ActorRef actor = source.toMat(sink, Keep.<ActorRef, NotUsed>left()).run(system); final ActorRef actor = source.toMat(sink, Keep.<ActorRef, NotUsed>left()).run(system);
probe.watch(actor); probe.watch(actor);
probe.expectTerminated(actor); probe.expectTerminated(actor);
@ -1378,7 +1389,7 @@ public class FlowTest extends StreamTest {
future.toCompletableFuture().complete(Flow.fromFunction((id) -> id)); future.toCompletableFuture().complete(Flow.fromFunction((id) -> id));
Integer result = Integer result =
Source.range(1, 10) Source.range(1, 10)
.via(Flow.lazyInitAsync(() -> future)) .via(Flow.lazyCompletionStageFlow(() -> future))
.runWith(Sink.<Integer>head(), system) .runWith(Sink.<Integer>head(), system)
.toCompletableFuture() .toCompletableFuture()
.get(3, TimeUnit.SECONDS); .get(3, TimeUnit.SECONDS);

View file

@ -221,10 +221,11 @@ public class GraphDslTest extends StreamTest {
@Test @Test
public void canUseMapMaterializedValueOnGraphs() { public void canUseMapMaterializedValueOnGraphs() {
Graph<SourceShape<Object>, NotUsed> srcGraph = Source.empty(); Graph<SourceShape<Object>, NotUsed> srcGraph = Source.empty();
Graph<SourceShape<Object>, Pair> mappedMatValueSrcGraph = Graph<SourceShape<Object>, Pair<NotUsed, NotUsed>> mappedMatValueSrcGraph =
Graph.mapMaterializedValue(srcGraph, notUsed -> new Pair(notUsed, notUsed)); Graph.mapMaterializedValue(
srcGraph, notUsed -> new Pair<NotUsed, NotUsed>(notUsed, notUsed));
Sink<Object, CompletionStage<Done>> snk = Sink.ignore(); Sink<Object, CompletionStage<Done>> snk = Sink.ignore();
Pair<NotUsed, NotUsed> pair = Source.fromGraph(mappedMatValueSrcGraph).to(snk).run(system); Pair<NotUsed, NotUsed> pair = Source.fromGraph(mappedMatValueSrcGraph).to(snk).run(system);
assertEquals(pair, new Pair(NotUsed.getInstance(), NotUsed.getInstance())); assertEquals(pair, new Pair<NotUsed, NotUsed>(NotUsed.getInstance(), NotUsed.getInstance()));
} }
} }

View file

@ -29,7 +29,7 @@ public class SetupTest extends StreamTest {
@Test @Test
public void shouldExposeMaterializerAndAttributesToSource() throws Exception { public void shouldExposeMaterializerAndAttributesToSource() throws Exception {
final Source<Pair<Boolean, Boolean>, CompletionStage<NotUsed>> source = final Source<Pair<Boolean, Boolean>, CompletionStage<NotUsed>> source =
Source.setup( Source.fromMaterializer(
(mat, attr) -> (mat, attr) ->
Source.single(Pair.create(mat.isShutdown(), attr.attributeList().isEmpty()))); Source.single(Pair.create(mat.isShutdown(), attr.attributeList().isEmpty())));
@ -41,7 +41,7 @@ public class SetupTest extends StreamTest {
@Test @Test
public void shouldExposeMaterializerAndAttributesToFlow() throws Exception { public void shouldExposeMaterializerAndAttributesToFlow() throws Exception {
final Flow<Object, Pair<Boolean, Boolean>, CompletionStage<NotUsed>> flow = final Flow<Object, Pair<Boolean, Boolean>, CompletionStage<NotUsed>> flow =
Flow.setup( Flow.fromMaterializer(
(mat, attr) -> (mat, attr) ->
Flow.fromSinkAndSource( Flow.fromSinkAndSource(
Sink.ignore(), Sink.ignore(),
@ -59,7 +59,7 @@ public class SetupTest extends StreamTest {
@Test @Test
public void shouldExposeMaterializerAndAttributesToSink() throws Exception { public void shouldExposeMaterializerAndAttributesToSink() throws Exception {
Sink<Object, CompletionStage<CompletionStage<Pair<Boolean, Boolean>>>> sink = Sink<Object, CompletionStage<CompletionStage<Pair<Boolean, Boolean>>>> sink =
Sink.setup( Sink.fromMaterializer(
(mat, attr) -> (mat, attr) ->
Sink.fold( Sink.fold(
Pair.create(mat.isShutdown(), attr.attributeList().isEmpty()), Keep.left())); Pair.create(mat.isShutdown(), attr.attributeList().isEmpty()), Keep.left()));

View file

@ -44,6 +44,7 @@ import static akka.stream.testkit.StreamTestKit.PublisherProbeSubscription;
import static akka.stream.testkit.TestPublisher.ManualProbe; import static akka.stream.testkit.TestPublisher.ManualProbe;
import static org.hamcrest.core.Is.is; import static org.hamcrest.core.Is.is;
import static org.junit.Assert.*; import static org.junit.Assert.*;
import static org.hamcrest.MatcherAssert.assertThat;
@SuppressWarnings("serial") @SuppressWarnings("serial")
public class SourceTest extends StreamTest { public class SourceTest extends StreamTest {
@ -327,7 +328,7 @@ public class SourceTest extends StreamTest {
List<Object> output = probe.receiveN(5); List<Object> output = probe.receiveN(5);
assertEquals(Arrays.asList(4, 3, 2, 1, 0), output); assertEquals(Arrays.asList(4, 3, 2, 1, 0), output);
probe.expectNoMessage(FiniteDuration.create(500, TimeUnit.MILLISECONDS)); probe.expectNoMessage(Duration.ofMillis(500));
} }
@Test @Test
@ -581,7 +582,7 @@ public class SourceTest extends StreamTest {
final Iterable<String> input = Arrays.asList("A", "B", "C"); final Iterable<String> input = Arrays.asList("A", "B", "C");
CompletionStage<String> future1 = Source.from(input).runWith(Sink.<String>head(), system); CompletionStage<String> future1 = Source.from(input).runWith(Sink.<String>head(), system);
CompletionStage<String> future2 = CompletionStage<String> future2 =
Source.fromCompletionStage(future1).runWith(Sink.<String>head(), system); Source.completionStage(future1).runWith(Sink.<String>head(), system);
String result = future2.toCompletableFuture().get(3, TimeUnit.SECONDS); String result = future2.toCompletableFuture().get(3, TimeUnit.SECONDS);
assertEquals("A", result); assertEquals("A", result);
} }
@ -590,7 +591,7 @@ public class SourceTest extends StreamTest {
public void mustWorkFromFutureVoid() throws Exception { public void mustWorkFromFutureVoid() throws Exception {
CompletionStage<Void> future = CompletableFuture.completedFuture(null); CompletionStage<Void> future = CompletableFuture.completedFuture(null);
CompletionStage<List<Void>> future2 = CompletionStage<List<Void>> future2 =
Source.fromCompletionStage(future).runWith(Sink.seq(), system); Source.completionStage(future).runWith(Sink.seq(), system);
List<Void> result = future2.toCompletableFuture().get(3, TimeUnit.SECONDS); List<Void> result = future2.toCompletableFuture().get(3, TimeUnit.SECONDS);
assertEquals(0, result.size()); assertEquals(0, result.size());
} }
@ -657,7 +658,12 @@ public class SourceTest extends StreamTest {
@Test @Test
public void mustBeAbleToUseActorRefSource() throws Exception { public void mustBeAbleToUseActorRefSource() throws Exception {
final TestKit probe = new TestKit(system); final TestKit probe = new TestKit(system);
final Source<Integer, ActorRef> actorRefSource = Source.actorRef(10, OverflowStrategy.fail()); final Source<Integer, ActorRef> actorRefSource =
Source.<Integer>actorRef(
msg -> Optional.<CompletionStrategy>empty(),
msg -> Optional.<Throwable>empty(),
10,
OverflowStrategy.fail());
final ActorRef ref = final ActorRef ref =
actorRefSource actorRefSource
.to( .to(
@ -799,7 +805,11 @@ public class SourceTest extends StreamTest {
if (elem == 1) throw new RuntimeException("ex"); if (elem == 1) throw new RuntimeException("ex");
else return elem; else return elem;
}) })
.recover(new PFBuilder<Throwable, Integer>().matchAny(ex -> 0).build()); .recoverWithRetries(
1,
new PFBuilder<Throwable, Source<Integer, NotUsed>>()
.matchAny(ex -> Source.single(0))
.build());
final CompletionStage<Done> future = final CompletionStage<Done> future =
source.runWith( source.runWith(
@ -867,6 +877,7 @@ public class SourceTest extends StreamTest {
probe.expectMsgAllOf(0, 1, 2, 3); probe.expectMsgAllOf(0, 1, 2, 3);
} }
@SuppressWarnings("unchecked")
@Test @Test
public void mustBeAbleToZipN() throws Exception { public void mustBeAbleToZipN() throws Exception {
final TestKit probe = new TestKit(system); final TestKit probe = new TestKit(system);
@ -895,7 +906,7 @@ public class SourceTest extends StreamTest {
final List<Source<Integer, ?>> sources = Arrays.asList(source1, source2); final List<Source<Integer, ?>> sources = Arrays.asList(source1, source2);
final Source<Boolean, ?> source = final Source<Boolean, ?> source =
Source.zipWithN(list -> new Boolean(list.contains(0)), sources); Source.zipWithN(list -> Boolean.valueOf(list.contains(0)), sources);
final CompletionStage<Done> future = final CompletionStage<Done> future =
source.runWith( source.runWith(

View file

@ -7,7 +7,6 @@ package akka.stream.typed.javadsl;
import akka.actor.typed.ActorRef; import akka.actor.typed.ActorRef;
import akka.actor.typed.ActorSystem; import akka.actor.typed.ActorSystem;
import akka.japi.JavaPartialFunction; import akka.japi.JavaPartialFunction;
import akka.stream.ActorMaterializer;
import akka.stream.OverflowStrategy; import akka.stream.OverflowStrategy;
import akka.stream.javadsl.Sink; import akka.stream.javadsl.Sink;
import akka.stream.javadsl.Source; import akka.stream.javadsl.Source;

View file

@ -7,7 +7,7 @@ package docs.akka.stream.typed;
// #actor-sink-ref // #actor-sink-ref
import akka.NotUsed; import akka.NotUsed;
import akka.actor.typed.ActorRef; import akka.actor.typed.ActorRef;
import akka.stream.ActorMaterializer; import akka.actor.typed.ActorSystem;
import akka.stream.javadsl.Sink; import akka.stream.javadsl.Sink;
import akka.stream.javadsl.Source; import akka.stream.javadsl.Source;
import akka.stream.typed.javadsl.ActorSink; import akka.stream.typed.javadsl.ActorSink;
@ -38,7 +38,7 @@ public class ActorSinkExample {
} }
// #actor-sink-ref // #actor-sink-ref
final ActorMaterializer mat = null; final ActorSystem<Void> system = null;
{ {
// #actor-sink-ref // #actor-sink-ref
@ -47,7 +47,7 @@ public class ActorSinkExample {
final Sink<Protocol, NotUsed> sink = ActorSink.actorRef(actor, new Complete(), Fail::new); final Sink<Protocol, NotUsed> sink = ActorSink.actorRef(actor, new Complete(), Fail::new);
Source.<Protocol>single(new Message("msg1")).runWith(sink, mat); Source.<Protocol>single(new Message("msg1")).runWith(sink, system);
// #actor-sink-ref // #actor-sink-ref
} }
} }

View file

@ -10,6 +10,12 @@ import scala.collection.immutable
object UniformFanInShape { object UniformFanInShape {
def apply[I, O](outlet: Outlet[O], inlets: Inlet[I]*): UniformFanInShape[I, O] = def apply[I, O](outlet: Outlet[O], inlets: Inlet[I]*): UniformFanInShape[I, O] =
new UniformFanInShape(inlets.size, FanInShape.Ports(outlet, inlets.toList)) new UniformFanInShape(inlets.size, FanInShape.Ports(outlet, inlets.toList))
/** Java API */
def create[I, O](outlet: Outlet[O], inlets: java.util.List[Inlet[I]]): UniformFanInShape[I, O] = {
import akka.util.ccompat.JavaConverters._
new UniformFanInShape(inlets.size, FanInShape.Ports(outlet, inlets.asScala.toList))
}
} }
class UniformFanInShape[-T, +O](val n: Int, _init: FanInShape.Init[O]) extends FanInShape[O](_init) { class UniformFanInShape[-T, +O](val n: Int, _init: FanInShape.Init[O]) extends FanInShape[O](_init) {

View file

@ -1514,7 +1514,6 @@ final class Flow[In, Out, Mat](delegate: scaladsl.Flow[In, Out, Mat]) extends Gr
* *
* '''Cancels when''' downstream cancels * '''Cancels when''' downstream cancels
*/ */
@deprecated("Use recoverWithRetries instead.", "2.4.4")
def recover(pf: PartialFunction[Throwable, Out]): javadsl.Flow[In, Out, Mat] = def recover(pf: PartialFunction[Throwable, Out]): javadsl.Flow[In, Out, Mat] =
new Flow(delegate.recover(pf)) new Flow(delegate.recover(pf))
@ -1533,7 +1532,6 @@ final class Flow[In, Out, Mat](delegate: scaladsl.Flow[In, Out, Mat]) extends Gr
* *
* '''Cancels when''' downstream cancels * '''Cancels when''' downstream cancels
*/ */
@deprecated("Use recoverWithRetries instead.", "2.4.4")
def recover(clazz: Class[_ <: Throwable], supplier: Supplier[Out]): javadsl.Flow[In, Out, Mat] = def recover(clazz: Class[_ <: Throwable], supplier: Supplier[Out]): javadsl.Flow[In, Out, Mat] =
recover { recover {
case elem if clazz.isInstance(elem) => supplier.get() case elem if clazz.isInstance(elem) => supplier.get()
@ -1628,7 +1626,10 @@ final class Flow[In, Out, Mat](delegate: scaladsl.Flow[In, Out, Mat]) extends Gr
* *
* '''Cancels when''' downstream cancels * '''Cancels when''' downstream cancels
* *
* @deprecated use `recoverWithRetries` instead
*/ */
@Deprecated
@deprecated("Use recoverWithRetries instead.", "2.6.6")
def recoverWith( def recoverWith(
clazz: Class[_ <: Throwable], clazz: Class[_ <: Throwable],
supplier: Supplier[Graph[SourceShape[Out], NotUsed]]): javadsl.Flow[In, Out, Mat] = supplier: Supplier[Graph[SourceShape[Out], NotUsed]]): javadsl.Flow[In, Out, Mat] =

View file

@ -1876,7 +1876,6 @@ final class Source[Out, Mat](delegate: scaladsl.Source[Out, Mat]) extends Graph[
* *
* '''Cancels when''' downstream cancels * '''Cancels when''' downstream cancels
*/ */
@deprecated("Use recoverWithRetries instead.", "2.4.4")
def recover(pf: PartialFunction[Throwable, Out]): javadsl.Source[Out, Mat] = def recover(pf: PartialFunction[Throwable, Out]): javadsl.Source[Out, Mat] =
new Source(delegate.recover(pf)) new Source(delegate.recover(pf))
@ -1895,7 +1894,6 @@ final class Source[Out, Mat](delegate: scaladsl.Source[Out, Mat]) extends Graph[
* *
* '''Cancels when''' downstream cancels * '''Cancels when''' downstream cancels
*/ */
@deprecated("Use recoverWithRetries instead.", "2.4.4")
def recover(clazz: Class[_ <: Throwable], supplier: Supplier[Out]): javadsl.Source[Out, Mat] = def recover(clazz: Class[_ <: Throwable], supplier: Supplier[Out]): javadsl.Source[Out, Mat] =
recover { recover {
case elem if clazz.isInstance(elem) => supplier.get() case elem if clazz.isInstance(elem) => supplier.get()
@ -1966,7 +1964,10 @@ final class Source[Out, Mat](delegate: scaladsl.Source[Out, Mat]) extends Graph[
* *
* '''Cancels when''' downstream cancels * '''Cancels when''' downstream cancels
* *
* @deprecated use `recoverWithRetries` instead
*/ */
@Deprecated
@deprecated("Use recoverWithRetries instead.", "2.6.6")
@silent("deprecated") @silent("deprecated")
def recoverWith(pf: PartialFunction[Throwable, _ <: Graph[SourceShape[Out], NotUsed]]): Source[Out, Mat] = def recoverWith(pf: PartialFunction[Throwable, _ <: Graph[SourceShape[Out], NotUsed]]): Source[Out, Mat] =
new Source(delegate.recoverWith(pf)) new Source(delegate.recoverWith(pf))
@ -1990,7 +1991,11 @@ final class Source[Out, Mat](delegate: scaladsl.Source[Out, Mat]) extends Graph[
* *
* '''Cancels when''' downstream cancels * '''Cancels when''' downstream cancels
* *
* @deprecated use `recoverWithRetries` instead
*/ */
@Deprecated
@deprecated("Use recoverWithRetries instead.", "2.6.6")
@silent("deprecated")
def recoverWith( def recoverWith(
clazz: Class[_ <: Throwable], clazz: Class[_ <: Throwable],
supplier: Supplier[Graph[SourceShape[Out], NotUsed]]): Source[Out, Mat] = supplier: Supplier[Graph[SourceShape[Out], NotUsed]]): Source[Out, Mat] =

View file

@ -19,6 +19,22 @@ object AkkaDisciplinePlugin extends AutoPlugin {
// We allow warnings in docs to get the 'snippets' right // We allow warnings in docs to get the 'snippets' right
val nonFatalWarningsFor = Set("akka-docs") val nonFatalWarningsFor = Set("akka-docs")
val nonFatalJavaWarningsFor = Set(
// for sun.misc.Unsafe and AbstractScheduler
"akka-actor",
// references to deprecated PARSER fields in generated message formats?
"akka-actor-typed-tests",
// references to deprecated PARSER fields in generated message formats?
"akka-cluster-typed",
// use of deprecated akka.protobuf.GeneratedMessage
"akka-protobuf",
// references to deprecated PARSER fields in generated message formats?
"akka-remote",
// references to deprecated PARSER fields in generated message formats?
"akka-distributed-data",
// references to deprecated PARSER fields in generated message formats?
"akka-cluster-sharding-typed",
)
val looseProjects = Set( val looseProjects = Set(
"akka-actor", "akka-actor",
@ -60,6 +76,10 @@ object AkkaDisciplinePlugin extends AutoPlugin {
else Seq.empty else Seq.empty
), ),
Test / scalacOptions --= testUndicipline, Test / scalacOptions --= testUndicipline,
Compile / javacOptions ++= (
if (!nonFatalJavaWarningsFor(name.value)) Seq("-Werror", "-Xlint:deprecation", "-Xlint:unchecked")
else Seq.empty
),
Compile / scalacOptions ++= (CrossVersion.partialVersion(scalaVersion.value) match { Compile / scalacOptions ++= (CrossVersion.partialVersion(scalaVersion.value) match {
case Some((2, 13)) => case Some((2, 13)) =>
disciplineScalacOptions -- Set( disciplineScalacOptions -- Set(

View file

@ -51,14 +51,14 @@ object TestExtras {
}, },
checkTestsHaveRun := { checkTestsHaveRun := {
require( def shouldExist(description: String, filename: String): Unit =
file("akka-stream-tests/target/test-reports/TEST-akka.stream.scaladsl.FlowPublisherSinkSpec.xml").exists, require(file(filename).exists, s"$description should be run as part of the build")
"The jdk9-only FlowPublisherSinkSpec.scala should be run as part of the build"
) List(
require( "The java JavaExtension.java" -> "akka-actor-tests/target/test-reports/akka.actor.JavaExtension.xml",
file("akka-stream-tests/target/test-reports/TEST-akka.stream.javadsl.JavaFlowSupportCompileTest.xml").exists, "The jdk9-only FlowPublisherSinkSpec.scala" -> "akka-stream-tests/target/test-reports/TEST-akka.stream.scaladsl.FlowPublisherSinkSpec.xml",
"The jdk9-only JavaFlowSupportCompileTest.java should be run as part of the build" "The jdk9-only JavaFlowSupportCompileTest.java" -> "akka-stream-tests/target/test-reports/TEST-akka.stream.javadsl.JavaFlowSupportCompileTest.xml",
) ).foreach((shouldExist _).tupled)
} }
) )
} }