diff --git a/akka-cluster-sharding-typed/src/main/scala/akka/cluster/sharding/typed/ActiveActiveShardingExtension.scala b/akka-cluster-sharding-typed/src/main/scala/akka/cluster/sharding/typed/ReplicatedShardingExtension.scala similarity index 68% rename from akka-cluster-sharding-typed/src/main/scala/akka/cluster/sharding/typed/ActiveActiveShardingExtension.scala rename to akka-cluster-sharding-typed/src/main/scala/akka/cluster/sharding/typed/ReplicatedShardingExtension.scala index 163dabe3e6..d650106857 100644 --- a/akka-cluster-sharding-typed/src/main/scala/akka/cluster/sharding/typed/ActiveActiveShardingExtension.scala +++ b/akka-cluster-sharding-typed/src/main/scala/akka/cluster/sharding/typed/ReplicatedShardingExtension.scala @@ -9,7 +9,7 @@ import akka.actor.typed.Extension import akka.actor.typed.ExtensionId import akka.annotation.ApiMayChange import akka.annotation.DoNotInherit -import akka.cluster.sharding.typed.internal.ActiveActiveShardingExtensionImpl +import akka.cluster.sharding.typed.internal.ReplicatedShardingExtensionImpl import akka.cluster.sharding.typed.scaladsl.EntityRef import akka.persistence.typed.ReplicaId import java.util.{ Map => JMap } @@ -17,16 +17,16 @@ import java.util.{ Map => JMap } import akka.actor.typed.ActorRef /** - * Extension for running active active in sharding by starting one separate instance of sharding per replica. + * Extension for running Replicated Event Sourcing in sharding by starting one separate instance of sharding per replica. * The sharding instances can be confined to datacenters or cluster roles or run on the same set of cluster nodes. */ @ApiMayChange -object ActiveActiveShardingExtension extends ExtensionId[ActiveActiveShardingExtension] { +object ReplicatedShardingExtension extends ExtensionId[ReplicatedShardingExtension] { - override def createExtension(system: ActorSystem[_]): ActiveActiveShardingExtension = - new ActiveActiveShardingExtensionImpl(system) + override def createExtension(system: ActorSystem[_]): ReplicatedShardingExtension = + new ReplicatedShardingExtensionImpl(system) - def get(system: ActorSystem[_]): ActiveActiveShardingExtension = apply(system) + def get(system: ActorSystem[_]): ReplicatedShardingExtension = apply(system) } @@ -35,27 +35,27 @@ object ActiveActiveShardingExtension extends ExtensionId[ActiveActiveShardingExt */ @DoNotInherit @ApiMayChange -trait ActiveActiveShardingExtension extends Extension { +trait ReplicatedShardingExtension extends Extension { /** - * Init one instance sharding per replica in the given settings and return a [[ActiveActiveSharding]] representing those. + * Init one instance sharding per replica in the given settings and return a [[ReplicatedSharding]] representing those. * - * @tparam M The type of messages the active active event sourced actor accepts + * @tparam M The type of messages the replicated event sourced actor accepts * @tparam E The type of envelope used for routing messages to actors, the same for all replicas * - * Note, multiple calls on the same node will not start new sharding instances but will return a new instance of [[ActiveActiveSharding]] + * Note, multiple calls on the same node will not start new sharding instances but will return a new instance of [[ReplicatedSharding]] */ - def init[M, E](settings: ActiveActiveShardingSettings[M, E]): ActiveActiveSharding[M, E] + def init[M, E](settings: ReplicatedShardingSettings[M, E]): ReplicatedSharding[M, E] } /** - * Represents the sharding instances for the replicas of one active active entity type + * Represents the sharding instances for the replicas of one replicated event sourcing entity type * * Not for user extension. */ @DoNotInherit @ApiMayChange -trait ActiveActiveSharding[M, E] { +trait ReplicatedSharding[M, E] { /** * Scala API: Returns the actor refs for the shard region or proxies of sharding for each replica for user defined diff --git a/akka-cluster-sharding-typed/src/main/scala/akka/cluster/sharding/typed/ActiveActiveShardingSettings.scala b/akka-cluster-sharding-typed/src/main/scala/akka/cluster/sharding/typed/ReplicatedShardingSettings.scala similarity index 74% rename from akka-cluster-sharding-typed/src/main/scala/akka/cluster/sharding/typed/ActiveActiveShardingSettings.scala rename to akka-cluster-sharding-typed/src/main/scala/akka/cluster/sharding/typed/ReplicatedShardingSettings.scala index d74d88aaf3..799e243960 100644 --- a/akka-cluster-sharding-typed/src/main/scala/akka/cluster/sharding/typed/ActiveActiveShardingSettings.scala +++ b/akka-cluster-sharding-typed/src/main/scala/akka/cluster/sharding/typed/ReplicatedShardingSettings.scala @@ -18,12 +18,12 @@ import akka.annotation.ApiMayChange import akka.cluster.sharding.typed.internal.EntityTypeKeyImpl @ApiMayChange -object ActiveActiveShardingSettings { +object ReplicatedShardingSettings { /** * Java API: * - * @tparam M The type of messages the active active entity accepts + * @tparam M The type of messages the replicated entity accepts * @tparam E The type for envelopes used for sending `M`s over sharding */ def create[M, E]( @@ -33,7 +33,7 @@ object ActiveActiveShardingSettings { JEntityTypeKey[M], ReplicaId, JSet[ReplicaId], - ReplicaSettings[M, E]]): ActiveActiveShardingSettings[M, E] = { + ReplicaSettings[M, E]]): ReplicatedShardingSettings[M, E] = { implicit val classTag: ClassTag[M] = ClassTag(messageClass) apply[M, E](allReplicaIds.asScala.toSet)((key, replica, _) => settingsPerReplicaFactory(key.asInstanceOf[EntityTypeKeyImpl[M]], replica, allReplicaIds)) @@ -42,13 +42,13 @@ object ActiveActiveShardingSettings { /** * Scala API: * - * @tparam M The type of messages the active active entity accepts + * @tparam M The type of messages the replicated entity accepts * @tparam E The type for envelopes used for sending `M`s over sharding */ def apply[M: ClassTag, E](allReplicaIds: Set[ReplicaId])( settingsPerReplicaFactory: (EntityTypeKey[M], ReplicaId, Set[ReplicaId]) => ReplicaSettings[M, E]) - : ActiveActiveShardingSettings[M, E] = { - new ActiveActiveShardingSettings(allReplicaIds.map { replicaId => + : ReplicatedShardingSettings[M, E] = { + new ReplicatedShardingSettings(allReplicaIds.map { replicaId => val typeKey = EntityTypeKey[M](replicaId.id) settingsPerReplicaFactory(typeKey, replicaId, allReplicaIds) }.toVector, directReplication = false) @@ -56,23 +56,23 @@ object ActiveActiveShardingSettings { } /** - * @tparam M The type of messages the active active entity accepts + * @tparam M The type of messages the replicated entity accepts * @tparam E The type for envelopes used for sending `M`s over sharding */ @ApiMayChange -final class ActiveActiveShardingSettings[M, E] private ( +final class ReplicatedShardingSettings[M, E] private ( val replicas: immutable.Seq[ReplicaSettings[M, E]], val directReplication: Boolean) { /** - * Start direct replication over sharding when active active sharding starts up, requires the entities + * Start direct replication over sharding when replicated sharding starts up, requires the entities * to also have it enabled through [[akka.persistence.typed.scaladsl.EventSourcedBehavior#withEventPublishing()]] - * or [[akka.persistence.typed.javadsl.ActiveActiveEventSourcedBehavior#withEventPublishing()]] + * or [[akka.persistence.typed.javadsl.ReplicatedEventSourcedBehavior#withEventPublishing()]] * to work. - + * */ - def withDirectReplication(): ActiveActiveShardingSettings[M, E] = - new ActiveActiveShardingSettings(replicas, directReplication = true) + def withDirectReplication(): ReplicatedShardingSettings[M, E] = + new ReplicatedShardingSettings(replicas, directReplication = true) } @@ -81,7 +81,7 @@ object ReplicaSettings { /** * Java API: Defines the [[akka.cluster.sharding.typed.javadsl.Entity]] to use for a given replica, note that the behavior - * can be a [[akka.persistence.typed.javadsl.ActiveActiveEventSourcedBehavior]] or an arbitrary non persistent + * can be a [[akka.persistence.typed.javadsl.ReplicatedEventSourcedBehavior]] or an arbitrary non persistent * [[akka.actor.typed.Behavior]] but must never be a regular [[akka.persistence.typed.javadsl.EventSourcedBehavior]] * as that requires a single writer and that would cause it to have multiple writers. */ @@ -90,7 +90,7 @@ object ReplicaSettings { /** * Scala API: Defines the [[akka.cluster.sharding.typed.scaladsl.Entity]] to use for a given replica, note that the behavior - * can be a behavior created with [[akka.persistence.typed.scaladsl.ActiveActiveEventSourcing]] or an arbitrary non persistent + * can be a behavior created with [[akka.persistence.typed.scaladsl.ReplicatedEventSourcing]] or an arbitrary non persistent * [[akka.actor.typed.Behavior]] but must never be a regular [[akka.persistence.typed.scaladsl.EventSourcedBehavior]] * as that requires a single writer and that would cause it to have multiple writers. */ @@ -99,7 +99,7 @@ object ReplicaSettings { } /** - * Settings for a specific replica id in active active sharding + * Settings for a specific replica id in replicated sharding */ @ApiMayChange final class ReplicaSettings[M, E] private (val replicaId: ReplicaId, val entity: Entity[M, E]) diff --git a/akka-cluster-sharding-typed/src/main/scala/akka/cluster/sharding/typed/ActiveActiveShardingDirectReplication.scala b/akka-cluster-sharding-typed/src/main/scala/akka/cluster/sharding/typed/ShardingDirectReplication.scala similarity index 89% rename from akka-cluster-sharding-typed/src/main/scala/akka/cluster/sharding/typed/ActiveActiveShardingDirectReplication.scala rename to akka-cluster-sharding-typed/src/main/scala/akka/cluster/sharding/typed/ShardingDirectReplication.scala index 7073984f3e..f667348a21 100644 --- a/akka-cluster-sharding-typed/src/main/scala/akka/cluster/sharding/typed/ActiveActiveShardingDirectReplication.scala +++ b/akka-cluster-sharding-typed/src/main/scala/akka/cluster/sharding/typed/ShardingDirectReplication.scala @@ -18,13 +18,13 @@ import akka.persistence.typed.ReplicaId import akka.util.ccompat.JavaConverters._ /** - * Used when sharding Active Active entities in multiple instances of sharding, for example one per DC in a Multi DC + * Used when sharding Replicated Event Sourced entities in multiple instances of sharding, for example one per DC in a Multi DC * Akka Cluster. * - * This actor should be started once on each node where Active Active entities will run (the same nodes that you start + * This actor should be started once on each node where Replicated Event Sourced entities will run (the same nodes that you start * sharding on). The entities should be set up with [[akka.persistence.typed.scaladsl.EventSourcedBehavior#withEventPublishing()]] - * or [[akka.persistence.typed.javadsl.ActiveActiveEventSourcedBehavior#withEventPublishing()]] - * If using [[ActiveActiveSharding]] the replication can be enabled through [[ActiveActiveShardingSettings#withDirectReplication()]] + * or [[akka.persistence.typed.javadsl.ReplicatedEventSourcedBehavior#withEventPublishing()]] + * If using [[ReplicatedSharding]] the replication can be enabled through [[ReplicatedShardingSettings#withDirectReplication()]] * instead of starting this actor manually. * * Subscribes to locally written events through the event stream and sends the seen events to all the sharded replicas @@ -36,7 +36,7 @@ import akka.util.ccompat.JavaConverters._ * by default and with a custom extractor since the envelopes are handled internally. */ @ApiMayChange -object ActiveActiveShardingDirectReplication { +object ShardingDirectReplication { /** * Not for user extension diff --git a/akka-cluster-sharding-typed/src/main/scala/akka/cluster/sharding/typed/internal/ActiveActiveShardingExtensionImpl.scala b/akka-cluster-sharding-typed/src/main/scala/akka/cluster/sharding/typed/internal/ReplicatedShardingExtensionImpl.scala similarity index 70% rename from akka-cluster-sharding-typed/src/main/scala/akka/cluster/sharding/typed/internal/ActiveActiveShardingExtensionImpl.scala rename to akka-cluster-sharding-typed/src/main/scala/akka/cluster/sharding/typed/internal/ReplicatedShardingExtensionImpl.scala index 9c57b9f960..1447c9e559 100644 --- a/akka-cluster-sharding-typed/src/main/scala/akka/cluster/sharding/typed/internal/ActiveActiveShardingExtensionImpl.scala +++ b/akka-cluster-sharding-typed/src/main/scala/akka/cluster/sharding/typed/internal/ReplicatedShardingExtensionImpl.scala @@ -10,9 +10,9 @@ import java.util.{ Map => JMap } import akka.actor.typed.ActorRef import akka.actor.typed.ActorSystem import akka.annotation.InternalApi -import akka.cluster.sharding.typed.ActiveActiveShardingExtension -import akka.cluster.sharding.typed.ActiveActiveSharding -import akka.cluster.sharding.typed.ActiveActiveShardingSettings +import akka.cluster.sharding.typed.ReplicatedShardingExtension +import akka.cluster.sharding.typed.ReplicatedSharding +import akka.cluster.sharding.typed.ReplicatedShardingSettings import akka.cluster.sharding.typed.scaladsl.ClusterSharding import akka.cluster.sharding.typed.scaladsl.EntityRef import akka.cluster.sharding.typed.scaladsl.EntityTypeKey @@ -20,7 +20,7 @@ import akka.persistence.typed.PersistenceId import akka.persistence.typed.ReplicaId import org.slf4j.LoggerFactory import akka.actor.typed.scaladsl.LoggerOps -import akka.cluster.sharding.typed.ActiveActiveShardingDirectReplication +import akka.cluster.sharding.typed.ShardingDirectReplication import akka.util.ccompat.JavaConverters._ @@ -28,19 +28,18 @@ import akka.util.ccompat.JavaConverters._ * INTERNAL API */ @InternalApi -private[akka] final class ActiveActiveShardingExtensionImpl(system: ActorSystem[_]) - extends ActiveActiveShardingExtension { +private[akka] final class ReplicatedShardingExtensionImpl(system: ActorSystem[_]) extends ReplicatedShardingExtension { private val counter = new AtomicLong(0) private val logger = LoggerFactory.getLogger(getClass) - override def init[M, E](settings: ActiveActiveShardingSettings[M, E]): ActiveActiveSharding[M, E] = { + override def init[M, E](settings: ReplicatedShardingSettings[M, E]): ReplicatedSharding[M, E] = { val sharding = ClusterSharding(system) val initializedReplicas = settings.replicas.map { replicaSettings => // start up a sharding instance per replica id logger.infoN( - "Starting Active Active sharding for replica [{}] (ShardType: [{}])", + "Starting Replicated Event Sourcing sharding for replica [{}] (ShardType: [{}])", replicaSettings.replicaId.id, replicaSettings.entity.typeKey.name) val regionOrProxy = sharding.init(replicaSettings.entity) @@ -50,14 +49,14 @@ private[akka] final class ActiveActiveShardingExtensionImpl(system: ActorSystem[ case (id, _, regionOrProxy) => id -> regionOrProxy }.toMap if (settings.directReplication) { - logger.infoN("Starting Active Active Direct Replication") + logger.infoN("Starting Replicated Event Sourcing Direct Replication") system.systemActorOf( - ActiveActiveShardingDirectReplication(replicaToRegionOrProxy), - s"activeActiveDirectReplication-${counter.incrementAndGet()}") + ShardingDirectReplication(replicaToRegionOrProxy), + s"directReplication-${counter.incrementAndGet()}") } val replicaToTypeKey = initializedReplicas.map { case (id, typeKey, _) => id -> typeKey }.toMap - new ActiveActiveShardingImpl(sharding, replicaToRegionOrProxy, replicaToTypeKey) + new ReplicatedShardingImpl(sharding, replicaToRegionOrProxy, replicaToTypeKey) } } @@ -65,11 +64,11 @@ private[akka] final class ActiveActiveShardingExtensionImpl(system: ActorSystem[ * INTERNAL API */ @InternalApi -private[akka] final class ActiveActiveShardingImpl[M, E]( +private[akka] final class ReplicatedShardingImpl[M, E]( sharding: ClusterSharding, shardingPerReplica: Map[ReplicaId, ActorRef[E]], replicaTypeKeys: Map[ReplicaId, EntityTypeKey[M]]) - extends ActiveActiveSharding[M, E] { + extends ReplicatedSharding[M, E] { override def shardingRefs: Map[ReplicaId, ActorRef[E]] = shardingPerReplica override def getShardingRefs: JMap[ReplicaId, ActorRef[E]] = shardingRefs.asJava diff --git a/akka-cluster-sharding-typed/src/test/java/akka/cluster/sharding/typed/ActiveActiveShardingTest.java b/akka-cluster-sharding-typed/src/test/java/akka/cluster/sharding/typed/ReplicatedShardingTest.java similarity index 79% rename from akka-cluster-sharding-typed/src/test/java/akka/cluster/sharding/typed/ActiveActiveShardingTest.java rename to akka-cluster-sharding-typed/src/test/java/akka/cluster/sharding/typed/ReplicatedShardingTest.java index 490f4c2606..d9cd4dcab9 100644 --- a/akka-cluster-sharding-typed/src/test/java/akka/cluster/sharding/typed/ActiveActiveShardingTest.java +++ b/akka-cluster-sharding-typed/src/test/java/akka/cluster/sharding/typed/ReplicatedShardingTest.java @@ -27,18 +27,16 @@ import org.junit.ClassRule; import org.junit.Rule; import org.junit.Test; import org.scalatestplus.junit.JUnitSuite; -import scala.util.Random; import java.util.*; import java.util.stream.Collectors; import static org.junit.Assert.assertEquals; -public class ActiveActiveShardingTest extends JUnitSuite { +public class ReplicatedShardingTest extends JUnitSuite { - static class MyActiveActiveStringSet - extends ActiveActiveEventSourcedBehavior< - MyActiveActiveStringSet.Command, String, Set> { + static class MyReplicatedStringSet + extends ReplicatedEventSourcedBehavior> { interface Command {} static class Add implements Command { @@ -67,16 +65,16 @@ public class ActiveActiveShardingTest extends JUnitSuite { static Behavior create( String entityId, ReplicaId replicaId, Set allReplicas) { - return ActiveActiveEventSourcing.withSharedJournal( + return ReplicatedEventSourcing.withSharedJournal( entityId, replicaId, allReplicas, PersistenceTestKitReadJournal.Identifier(), - MyActiveActiveStringSet::new); + MyReplicatedStringSet::new); } - private MyActiveActiveStringSet(ActiveActiveContext activeActiveContext) { - super(activeActiveContext); + private MyReplicatedStringSet(ReplicationContext replicationContext) { + super(replicationContext); } @Override @@ -116,9 +114,9 @@ public class ActiveActiveShardingTest extends JUnitSuite { public static final class ForwardToRandom implements Command { public final String entityId; - public final MyActiveActiveStringSet.Command message; + public final MyReplicatedStringSet.Command message; - public ForwardToRandom(String entityId, MyActiveActiveStringSet.Command message) { + public ForwardToRandom(String entityId, MyReplicatedStringSet.Command message) { this.entityId = entityId; this.message = message; } @@ -126,9 +124,9 @@ public class ActiveActiveShardingTest extends JUnitSuite { public static final class ForwardToAll implements Command { public final String entityId; - public final MyActiveActiveStringSet.Command message; + public final MyReplicatedStringSet.Command message; - public ForwardToAll(String entityId, MyActiveActiveStringSet.Command message) { + public ForwardToAll(String entityId, MyReplicatedStringSet.Command message) { this.entityId = entityId; this.message = message; } @@ -144,19 +142,19 @@ public class ActiveActiveShardingTest extends JUnitSuite { Arrays.asList( new ReplicaId("DC-A"), new ReplicaId("DC-B"), new ReplicaId("DC-C")))); - private final ActiveActiveSharding< - MyActiveActiveStringSet.Command, ShardingEnvelope> + private final ReplicatedSharding< + MyReplicatedStringSet.Command, ShardingEnvelope> aaSharding; private ProxyActor(ActorContext context) { super(context); // #bootstrap - ActiveActiveShardingSettings< - MyActiveActiveStringSet.Command, ShardingEnvelope> + ReplicatedShardingSettings< + MyReplicatedStringSet.Command, ShardingEnvelope> aaShardingSettings = - ActiveActiveShardingSettings.create( - MyActiveActiveStringSet.Command.class, + ReplicatedShardingSettings.create( + MyReplicatedStringSet.Command.class, ALL_REPLICAS, // factory for replica settings for a given replica (entityTypeKey, replicaId, allReplicas) -> @@ -168,7 +166,7 @@ public class ActiveActiveShardingTest extends JUnitSuite { entityTypeKey, entityContext -> // factory for the entity for a given entity in that replica - MyActiveActiveStringSet.create( + MyReplicatedStringSet.create( entityContext.getEntityId(), replicaId, allReplicas)) // potentially use replica id as role or dc in Akka multi dc for the // sharding instance @@ -176,8 +174,8 @@ public class ActiveActiveShardingTest extends JUnitSuite { // .withDataCenter(replicaId.id())) .withRole(replicaId.id()))); - ActiveActiveShardingExtension extension = - ActiveActiveShardingExtension.get(getContext().getSystem()); + ReplicatedShardingExtension extension = + ReplicatedShardingExtension.get(getContext().getSystem()); aaSharding = extension.init(aaShardingSettings); // #bootstrap } @@ -191,7 +189,7 @@ public class ActiveActiveShardingTest extends JUnitSuite { } private Behavior onForwardToRandom(ForwardToRandom forwardToRandom) { - Map> refs = + Map> refs = aaSharding.getEntityRefsFor(forwardToRandom.entityId); int chosenIdx = new java.util.Random().nextInt(refs.size()); new ArrayList<>(refs.values()).get(chosenIdx).tell(forwardToRandom.message); @@ -200,7 +198,7 @@ public class ActiveActiveShardingTest extends JUnitSuite { private Behavior onForwardToAll(ForwardToAll forwardToAll) { // #all-entity-refs - Map> refs = + Map> refs = aaSharding.getEntityRefsFor(forwardToAll.entityId); refs.forEach((replicaId, ref) -> ref.tell(forwardToAll.message)); // #all-entity-refs @@ -238,16 +236,16 @@ public class ActiveActiveShardingTest extends JUnitSuite { // forward messages to replicas ActorRef proxy = testKit.spawn(ProxyActor.create()); - proxy.tell(new ProxyActor.ForwardToAll("id1", new MyActiveActiveStringSet.Add("to-all"))); - proxy.tell(new ProxyActor.ForwardToRandom("id1", new MyActiveActiveStringSet.Add("to-random"))); + proxy.tell(new ProxyActor.ForwardToAll("id1", new MyReplicatedStringSet.Add("to-all"))); + proxy.tell(new ProxyActor.ForwardToRandom("id1", new MyReplicatedStringSet.Add("to-random"))); testProbe.awaitAssert( () -> { - TestProbe responseProbe = testKit.createTestProbe(); + TestProbe responseProbe = testKit.createTestProbe(); proxy.tell( new ProxyActor.ForwardToAll( - "id1", new MyActiveActiveStringSet.GetTexts(responseProbe.ref()))); - List responses = responseProbe.receiveSeveralMessages(3); + "id1", new MyReplicatedStringSet.GetTexts(responseProbe.ref()))); + List responses = responseProbe.receiveSeveralMessages(3); Set uniqueTexts = responses.stream().flatMap(res -> res.texts.stream()).collect(Collectors.toSet()); assertEquals(new HashSet<>(Arrays.asList("to-all", "to-random")), uniqueTexts); diff --git a/akka-cluster-sharding-typed/src/test/scala/akka/cluster/sharding/typed/ActiveActiveShardingDirectReplicationSpec.scala b/akka-cluster-sharding-typed/src/test/scala/akka/cluster/sharding/typed/ReplicatedShardingDirectReplicationSpec.scala similarity index 85% rename from akka-cluster-sharding-typed/src/test/scala/akka/cluster/sharding/typed/ActiveActiveShardingDirectReplicationSpec.scala rename to akka-cluster-sharding-typed/src/test/scala/akka/cluster/sharding/typed/ReplicatedShardingDirectReplicationSpec.scala index bb85730a88..1cd99bb3a2 100644 --- a/akka-cluster-sharding-typed/src/test/scala/akka/cluster/sharding/typed/ActiveActiveShardingDirectReplicationSpec.scala +++ b/akka-cluster-sharding-typed/src/test/scala/akka/cluster/sharding/typed/ReplicatedShardingDirectReplicationSpec.scala @@ -15,12 +15,9 @@ import akka.persistence.typed.PublishedEvent import akka.persistence.typed.internal.{ PublishedEventImpl, ReplicatedPublishedEventMetaData, VersionVector } import akka.persistence.typed.ReplicaId -class ActiveActiveShardingDirectReplicationSpec - extends ScalaTestWithActorTestKit - with AnyWordSpecLike - with LogCapturing { +class ReplicatedShardingDirectReplicationSpec extends ScalaTestWithActorTestKit with AnyWordSpecLike with LogCapturing { - "Active active sharding replication" must { + "Replicated sharding direct replication" must { "replicate published events to all sharding proxies" in { val replicaAProbe = createTestProbe[ShardingEnvelope[PublishedEvent]]() @@ -28,7 +25,7 @@ class ActiveActiveShardingDirectReplicationSpec val replicaCProbe = createTestProbe[ShardingEnvelope[PublishedEvent]]() val replicationActor = spawn( - ActiveActiveShardingDirectReplication( + ShardingDirectReplication( typed.ReplicaId("ReplicaA"), replicaShardingProxies = Map( ReplicaId("ReplicaA") -> replicaAProbe.ref, @@ -36,7 +33,7 @@ class ActiveActiveShardingDirectReplicationSpec ReplicaId("ReplicaC") -> replicaCProbe.ref))) val upProbe = createTestProbe[Done]() - replicationActor ! ActiveActiveShardingDirectReplication.VerifyStarted(upProbe.ref) + replicationActor ! ShardingDirectReplication.VerifyStarted(upProbe.ref) upProbe.receiveMessage() // not bullet proof wrt to subscription being complete but good enough val event = PublishedEventImpl( diff --git a/akka-cluster-sharding-typed/src/test/scala/akka/cluster/sharding/typed/ActiveActiveShardingSpec.scala b/akka-cluster-sharding-typed/src/test/scala/akka/cluster/sharding/typed/ReplicatedShardingSpec.scala similarity index 76% rename from akka-cluster-sharding-typed/src/test/scala/akka/cluster/sharding/typed/ActiveActiveShardingSpec.scala rename to akka-cluster-sharding-typed/src/test/scala/akka/cluster/sharding/typed/ReplicatedShardingSpec.scala index 74c7f3052a..d87a13673f 100644 --- a/akka-cluster-sharding-typed/src/test/scala/akka/cluster/sharding/typed/ActiveActiveShardingSpec.scala +++ b/akka-cluster-sharding-typed/src/test/scala/akka/cluster/sharding/typed/ReplicatedShardingSpec.scala @@ -16,7 +16,7 @@ import akka.cluster.typed.Join import akka.persistence.testkit.PersistenceTestKitPlugin import akka.persistence.testkit.query.scaladsl.PersistenceTestKitReadJournal import akka.persistence.typed.ReplicaId -import akka.persistence.typed.scaladsl.ActiveActiveEventSourcing +import akka.persistence.typed.scaladsl.ReplicatedEventSourcing import akka.persistence.typed.scaladsl.Effect import akka.persistence.typed.scaladsl.EventSourcedBehavior import akka.serialization.jackson.CborSerializable @@ -25,7 +25,7 @@ import org.scalatest.wordspec.AnyWordSpecLike import scala.util.Random -object ActiveActiveShardingSpec { +object ReplicatedShardingSpec { def config = ConfigFactory.parseString(""" akka.loglevel = DEBUG akka.loggers = ["akka.testkit.SilenceAllTestEventListener"] @@ -36,19 +36,19 @@ object ActiveActiveShardingSpec { akka.remote.artery.canonical.port = 0""").withFallback(PersistenceTestKitPlugin.config) } -class ActiveActiveShardingSpec - extends ScalaTestWithActorTestKit(ActiveActiveShardingSpec.config) +class ReplicatedShardingSpec + extends ScalaTestWithActorTestKit(ReplicatedShardingSpec.config) with AnyWordSpecLike with LogCapturing { - object MyActiveActiveStringSet { + object MyReplicatedStringSet { trait Command extends CborSerializable case class Add(text: String) extends Command case class GetTexts(replyTo: ActorRef[Texts]) extends Command case class Texts(texts: Set[String]) extends CborSerializable def apply(entityId: String, replicaId: ReplicaId, allReplicas: Set[ReplicaId]): Behavior[Command] = - ActiveActiveEventSourcing.withSharedJournal( + ReplicatedEventSourcing.withSharedJournal( entityId, replicaId, allReplicas, @@ -70,15 +70,13 @@ class ActiveActiveShardingSpec object ProxyActor { sealed trait Command - case class ForwardToRandom(entityId: String, msg: MyActiveActiveStringSet.Command) extends Command - case class ForwardToAll(entityId: String, msg: MyActiveActiveStringSet.Command) extends Command + case class ForwardToRandom(entityId: String, msg: MyReplicatedStringSet.Command) extends Command + case class ForwardToAll(entityId: String, msg: MyReplicatedStringSet.Command) extends Command def apply(): Behavior[Command] = Behaviors.setup { context => // #bootstrap val aaShardingSettings = - ActiveActiveShardingSettings[ - MyActiveActiveStringSet.Command, - ShardingEnvelope[MyActiveActiveStringSet.Command]]( + ReplicatedShardingSettings[MyReplicatedStringSet.Command, ShardingEnvelope[MyReplicatedStringSet.Command]]( // all replicas Set(ReplicaId("DC-A"), ReplicaId("DC-B"), ReplicaId("DC-C"))) { (entityTypeKey, replicaId, allReplicaIds) => // factory for replica settings for a given replica @@ -87,7 +85,7 @@ class ActiveActiveShardingSpec // use the provided entity type key for sharding to get one sharding instance per replica Entity(entityTypeKey) { entityContext => // factory for the entity for a given entity in that replica - MyActiveActiveStringSet(entityContext.entityId, replicaId, allReplicaIds) + MyReplicatedStringSet(entityContext.entityId, replicaId, allReplicaIds) } // potentially use replica id as role or dc in Akka multi dc for the sharding instance // to control where replicas will live @@ -95,7 +93,7 @@ class ActiveActiveShardingSpec .withRole(replicaId.id)) } - val aaSharding = ActiveActiveShardingExtension(context.system).init(aaShardingSettings) + val aaSharding = ReplicatedShardingExtension(context.system).init(aaShardingSettings) // #bootstrap Behaviors.receiveMessage { @@ -115,7 +113,7 @@ class ActiveActiveShardingSpec } } - "Active active sharding" should { + "Replicated sharding" should { "form a one node cluster" in { val node = Cluster(system) @@ -128,13 +126,13 @@ class ActiveActiveShardingSpec "forward to replicas" in { val proxy = spawn(ProxyActor()) - proxy ! ProxyActor.ForwardToAll("id1", MyActiveActiveStringSet.Add("to-all")) - proxy ! ProxyActor.ForwardToRandom("id1", MyActiveActiveStringSet.Add("to-random")) + proxy ! ProxyActor.ForwardToAll("id1", MyReplicatedStringSet.Add("to-all")) + proxy ! ProxyActor.ForwardToRandom("id1", MyReplicatedStringSet.Add("to-random")) eventually { - val probe = createTestProbe[MyActiveActiveStringSet.Texts]() - proxy ! ProxyActor.ForwardToAll("id1", MyActiveActiveStringSet.GetTexts(probe.ref)) - val responses: Seq[MyActiveActiveStringSet.Texts] = probe.receiveMessages(3) + val probe = createTestProbe[MyReplicatedStringSet.Texts]() + proxy ! ProxyActor.ForwardToAll("id1", MyReplicatedStringSet.GetTexts(probe.ref)) + val responses: Seq[MyReplicatedStringSet.Texts] = probe.receiveMessages(3) val uniqueTexts = responses.flatMap(res => res.texts).toSet uniqueTexts should ===(Set("to-all", "to-random")) } diff --git a/akka-docs/src/main/paradox/project/examples.md b/akka-docs/src/main/paradox/project/examples.md index af9e879ffe..d790e8590e 100644 --- a/akka-docs/src/main/paradox/project/examples.md +++ b/akka-docs/src/main/paradox/project/examples.md @@ -61,11 +61,7 @@ from the events, or publish the events to other services. ## Multi-DC Persistence -@java[@extref[Multi-DC Persistence example project](samples:akka-samples-persistence-dc-java)] -@scala[@extref[Multi-DC Persistence example project](samples:akka-samples-persistence-dc-scala)] - -Illustrates how to use Lightbend's [Multi-DC Persistence](https://doc.akka.io/docs/akka-enhancements/current/persistence-dc/index.html) -with active-active persistent entities across data centers. +This commercial feature has now been superseeded by @ref[Replicated Event Sourcing](../typed/replicated-eventsourcing.md) ## Cluster with Docker diff --git a/akka-docs/src/main/paradox/typed/cluster-dc.md b/akka-docs/src/main/paradox/typed/cluster-dc.md index 8d512cb7af..ca0003bb71 100644 --- a/akka-docs/src/main/paradox/typed/cluster-dc.md +++ b/akka-docs/src/main/paradox/typed/cluster-dc.md @@ -192,8 +192,8 @@ other data centers. Especially when used together with Akka Persistence that is based on the single-writer principle it is important to avoid running the same entity at multiple locations at the same time with a shared data store. That would result in corrupt data since the events stored by different instances -may be interleaved and would be interpreted differently in a later replay. For active active persistent -entities see Lightbend's [Multi-DC Persistence](https://doc.akka.io/docs/akka-enhancements/current/persistence-dc/index.html) +may be interleaved and would be interpreted differently in a later replay. For replicated persistent +entities see @ref[Replciated Event Sourcing](replicated-eventsourcing.md). If you need global entities you have to pick one data center to host that entity type and only start `ClusterSharding` on nodes of that data center. If the data center is unreachable from another data center the diff --git a/akka-docs/src/main/paradox/typed/index-persistence.md b/akka-docs/src/main/paradox/typed/index-persistence.md index a3da54c018..2a7fd09aa5 100644 --- a/akka-docs/src/main/paradox/typed/index-persistence.md +++ b/akka-docs/src/main/paradox/typed/index-persistence.md @@ -9,7 +9,7 @@ project.description: Event Sourcing with Akka Persistence enables actors to pers @@@ index * [persistence](persistence.md) -* [active-active](persistence-active-active.md) +* [active-active](replicated-eventsourcing.md) * [cqrs](cqrs.md) * [persistence-style](persistence-style.md) * [persistence-snapshot](persistence-snapshot.md) @@ -20,6 +20,6 @@ project.description: Event Sourcing with Akka Persistence enables actors to pers * [persistence-query-leveldb](../persistence-query-leveldb.md) * [persistence-plugins](../persistence-plugins.md) * [persistence-journals](../persistence-journals.md) -* [active-active-examples](persistence-active-active-examples.md) +* [active-active-examples](replicated-eventsourcing-examples.md) @@@ diff --git a/akka-docs/src/main/paradox/typed/persistence-active-active-examples.md b/akka-docs/src/main/paradox/typed/replicated-eventsourcing-examples.md similarity index 82% rename from akka-docs/src/main/paradox/typed/persistence-active-active-examples.md rename to akka-docs/src/main/paradox/typed/replicated-eventsourcing-examples.md index 782557c848..d7b49e92eb 100644 --- a/akka-docs/src/main/paradox/typed/persistence-active-active-examples.md +++ b/akka-docs/src/main/paradox/typed/replicated-eventsourcing-examples.md @@ -1,6 +1,6 @@ -# Active-Active Examples +# Replicated Event Sourcing Examples -The following are more realistic examples of building systems with active-active event sourcing. +The following are more realistic examples of building systems with Replicated Event Sourcing. ## Auction @@ -16,21 +16,21 @@ We are building a small auction service. It has the following operations: We model those operations as commands to be sent to the auction actor: Scala -: @@snip [AuctionExample](/akka-persistence-typed-tests/src/test/scala/docs/akka/persistence/typed/AAAuctionExampleSpec.scala) { #commands } +: @@snip [AuctionExample](/akka-persistence-typed-tests/src/test/scala/docs/akka/persistence/typed/ReplicatedAuctionExampleSpec.scala) { #commands } The events: Scala -: @@snip [AuctionExample](/akka-persistence-typed-tests/src/test/scala/docs/akka/persistence/typed/AAAuctionExampleSpec.scala) { #events } +: @@snip [AuctionExample](/akka-persistence-typed-tests/src/test/scala/docs/akka/persistence/typed/ReplicatedAuctionExampleSpec.scala) { #events } The winner does not have to pay the highest bid but only enough to beat the second highest so the `highestCounterOffer` is in the `AuctionFinished` event. Let's have a look at the auction entity that will handle incoming commands: Scala -: @@snip [AuctionExample](/akka-persistence-typed-tests/src/test/scala/docs/akka/persistence/typed/AAAuctionExampleSpec.scala) { #command-handler } +: @@snip [AuctionExample](/akka-persistence-typed-tests/src/test/scala/docs/akka/persistence/typed/ReplicatedAuctionExampleSpec.scala) { #command-handler } -There is nothing specific to active-active about the command handler. It is the same as a command handler for a standard `EventSourcedBehavior`. +There is nothing specific to Replicated Event Sourcing about the command handler. It is the same as a command handler for a standard `EventSourcedBehavior`. For `OfferBid` and `AuctionFinished` we do nothing more than to emit events corresponding to the command. For `GetHighestBid` we respond with details from the state. Note, that we overwrite the actual offer of the highest bid here with the amount of the `highestCounterOffer`. This is done to follow the popular auction style where @@ -41,13 +41,13 @@ The initial state is taken from a `AuctionSetup` instance. The minimum bid is mo an `initialBid`. Scala -: @@snip [AuctionExample](/akka-persistence-typed-tests/src/test/scala/docs/akka/persistence/typed/AAAuctionExampleSpec.scala) { #setup } +: @@snip [AuctionExample](/akka-persistence-typed-tests/src/test/scala/docs/akka/persistence/typed/ReplicatedAuctionExampleSpec.scala) { #setup } The auction moves through the following phases: Scala -: @@snip [AuctionExample](/akka-persistence-typed-tests/src/test/scala/docs/akka/persistence/typed/AAAuctionExampleSpec.scala) { #phase } +: @@snip [AuctionExample](/akka-persistence-typed-tests/src/test/scala/docs/akka/persistence/typed/ReplicatedAuctionExampleSpec.scala) { #phase } The closing and closed states are to model waiting for all replicas to see the result of the auction before actually closing the action. @@ -56,7 +56,7 @@ Let's have a look at our state class, `AuctionState` which also represents the C Scala -: @@snip [AuctionExample](/akka-persistence-typed-tests/src/test/scala/docs/akka/persistence/typed/AAAuctionExampleSpec.scala) { #state } +: @@snip [AuctionExample](/akka-persistence-typed-tests/src/test/scala/docs/akka/persistence/typed/ReplicatedAuctionExampleSpec.scala) { #state } The state consists of a flag that keeps track of whether the auction is still active, the currently highest bid, and the highest counter offer so far. @@ -95,9 +95,9 @@ all replicas have seen all bids. In the event handler above, when recovery is not running, it calls `eventTriggers`. Scala -: @@snip [AuctionExample](/akka-persistence-typed-tests/src/test/scala/docs/akka/persistence/typed/AAAuctionExampleSpec.scala) { #event-triggers } +: @@snip [AuctionExample](/akka-persistence-typed-tests/src/test/scala/docs/akka/persistence/typed/ReplicatedAuctionExampleSpec.scala) { #event-triggers } -The event trigger uses the `ActiveActiveContext` to decide when to trigger the Finish of the action. +The event trigger uses the `ReplicationContext` to decide when to trigger the Finish of the action. When a replica saves the `AuctionFinished` event it checks whether it should close the auction. For the close to happen the replica must be the one designated to close and all replicas must have reported that they have finished. diff --git a/akka-docs/src/main/paradox/typed/persistence-active-active.md b/akka-docs/src/main/paradox/typed/replicated-eventsourcing.md similarity index 69% rename from akka-docs/src/main/paradox/typed/persistence-active-active.md rename to akka-docs/src/main/paradox/typed/replicated-eventsourcing.md index 4d621b2b31..df09cd5049 100644 --- a/akka-docs/src/main/paradox/typed/persistence-active-active.md +++ b/akka-docs/src/main/paradox/typed/replicated-eventsourcing.md @@ -1,4 +1,4 @@ -# Active-Active Event Sourcing +# Replicated Event Sourcing @@@ warning @@ -13,7 +13,7 @@ warning or deprecation period. It is also not recommended to use this module in This restriction means that in the event of network partitions, and for a short time during rolling re-deploys, `EventSourcedBehaviors`s are unavailable. -Active-active event sourcing enables running multiple replicas of each entity. +Replicated Event Sourcing enables running multiple replicas of each entity. There is automatic replication of every event persisted to all replicas. For instance, a replica can be run per: @@ -27,27 +27,27 @@ The motivations are: * Serve requests from a location near the user to provide better responsiveness * Balance the load over many servers -However, the event handler must be able to **handle concurrent events** as when active-active is enabled +However, the event handler must be able to **handle concurrent events** as when replication is enabled there is no longer the single writer principle as there is with a normal `EventSourcedBehavior`. -The state of an active-active `EventSourcedBehavior` is **eventually consistent**. Event replication may be delayed +The state of a replicated `EventSourcedBehavior` is **eventually consistent**. Event replication may be delayed due to network partitions and outages and the event handler and those reading the state must be designed to handle this. -To be able to use active active the journal and snapshot store used is required to have specific support for the metadata that active active needs (see @ref[Journal Support](#journal-support)) +To be able to use Replicated Event Sourcing the journal and snapshot store used is required to have specific support for the metadata that the replication needs (see @ref[Journal Support](#journal-support)) ## Relaxing the single writer principle for availability -Taking the example of using active-active to run a replica per data center. +Taking the example of using Replicated Event Sourcing to run a replica per data center. -When there is no network partitions and no concurrent writes the events stored by an `EventSourcedBehavior` at one replica can be replicated and consumed by another (corresponding) replica in another data center without any concerns. Such replicated events can simply be applied to the local state. +When there is no network partitions and no concurrent writes the events stored by a `EventSourcedBehavior` at one replica can be replicated and consumed by another (corresponding) replica in another data center without any concerns. Such replicated events can simply be applied to the local state. ![images/replicated-events1.png](images/replicated-events1.png) -The interesting part begins when there are concurrent writes by `EventSourcedBehavior`replicas. That is more likely to happen when there is a network partition, but it can also happen when there are no network issues. They simply write at the "same time" before the events from the other side have been replicated and consumed. +The interesting part begins when there are concurrent writes by `EventSourcedBehavior` replicas. That is more likely to happen when there is a network partition, but it can also happen when there are no network issues. They simply write at the "same time" before the events from the other side have been replicated and consumed. ![images/replicated-events2.png](images/replicated-events2.png) -The event handler logic for applying events to the state of the entity must be aware of that such concurrent updates can occur and it must be modeled to handle such conflicts. This means that it should typically have the same characteristics as a Conflict Free Replicated Data Type (CRDT). With a CRDT there are by definition no conflicts and the events can just be applied. The library provides some general purpose CRDTs, but the logic of how to apply events can also be defined by an application specific function. +The event handler logic for applying events to the state of the entity must be aware of that such concurrent updates can occur, and it must be modeled to handle such conflicts. This means that it should typically have the same characteristics as a Conflict Free Replicated Data Type (CRDT). With a CRDT there are by definition no conflicts, the events can always be applied. The library provides some general purpose CRDTs, but the logic of how to apply events can also be defined by an application specific function. For example, sometimes it's enough to use application specific timestamps to decide which update should win. @@ -58,44 +58,44 @@ To assist in implementing the event handler active-active detects these conflict @scala[The same API as regular `EventSourcedBehavior`s]@java[A very similar API to the regular `EventSourcedBehavior`] is used to define the logic. To enable an entity for active-active -replication @java[let it extend `ActiveActiveEventSourcedBehavior` instead of `EventSourcedBehavior` and] use the factory methods on @apidoc[ActiveActiveEventSourcing]. +replication @java[let it extend `ReplicatedEventSourcedBehavior` instead of `EventSourcedBehavior` and] use the factory methods on @scala[`akka.persistence.typed.scaladsl.ReplicatedEventSourcing`]@java[`akka.persistence.typed.javadsl.ReplicatedEventSourcing`]. All replicas need to be known up front: Scala -: @@snip [ActiveActiveCompileOnlySpec.scala](/akka-persistence-typed-tests/src/test/scala/docs/akka/persistence/typed/ActiveActiveCompileOnlySpec.scala) { #replicas } +: @@snip [ReplicatedEventSourcingCompileOnlySpec.scala](/akka-persistence-typed-tests/src/test/scala/docs/akka/persistence/typed/ReplicatedEventSourcingCompileOnlySpec.scala) { #replicas } Java -: @@snip [ActiveActiveCompileOnlyTest.java](/akka-persistence-typed-tests/src/test/java/jdocs/akka/persistence/typed/ActiveActiveCompileOnlyTest.java) { #replicas } +: @@snip [ReplicatedEventSourcingCompileOnlyTest.java](/akka-persistence-typed-tests/src/test/java/jdocs/akka/persistence/typed/ReplicatedEventSourcingCompileOnlyTest.java) { #replicas } Then to enable replication create the event sourced behavior with the factory method: Scala -: @@snip [ActiveActiveCompileOnlySpec.scala](/akka-persistence-typed-tests/src/test/scala/docs/akka/persistence/typed/ActiveActiveCompileOnlySpec.scala) { #factory } +: @@snip [ReplicatedEventSourcingCompileOnlySpec.scala](/akka-persistence-typed-tests/src/test/scala/docs/akka/persistence/typed/ReplicatedEventSourcingCompileOnlySpec.scala) { #factory } Java -: @@snip [ActiveActiveCompileOnlyTest.java](/akka-persistence-typed-tests/src/test/java/jdocs/akka/persistence/typed/ActiveActiveCompileOnlyTest.java) { #factory } +: @@snip [ReplicatedEventSourcingCompileOnlyTest.java](/akka-persistence-typed-tests/src/test/java/jdocs/akka/persistence/typed/ReplicatedEventSourcingCompileOnlyTest.java) { #factory } The factory takes in: * EntityID: this will be used as part of the underlying persistenceId * Replica: Which replica this instance is * All Replicas and the query plugin used to read their events -* A factory function to create an instance of the @scala[`EventSourcedBehavior`]@java[`ActiveActiveEventSourcedBehavior`] +* A factory function to create an instance of the @scala[`EventSourcedBehavior`]@java[`ReplicatedEventSourcedBehavior`] In this scenario each replica reads from each other's database effectively providing cross region replication for any database that has an Akka Persistence plugin. Alternatively if all the replicas use the same journal, e.g. for testing or if it is a distributed database such as Cassandra, the `withSharedJournal` factory can be used. Scala -: @@snip [ActiveActiveCompileOnlySpec.scala](/akka-persistence-typed-tests/src/test/scala/docs/akka/persistence/typed/ActiveActiveCompileOnlySpec.scala) { #factory-shared} +: @@snip [ReplicatedEventSourcingCompileOnlySpec.scala](/akka-persistence-typed-tests/src/test/scala/docs/akka/persistence/typed/ReplicatedEventSourcingCompileOnlySpec.scala) { #factory-shared} Java -: @@snip [ActiveActiveCompileOnlyTest.java](/akka-persistence-typed-tests/src/test/java/jdocs/akka/persistence/typed/ActiveActiveCompileOnlyTest.java) { #factory-shared } +: @@snip [ReplicatedEventSourcingCompileOnlyTest.java](/akka-persistence-typed-tests/src/test/java/jdocs/akka/persistence/typed/ReplicatedEventSourcingCompileOnlyTest.java) { #factory-shared } @@@ div { .group-scala } -The function passed to both factory methods return an `EventSourcedBehavior` and provide access to the @apidoc[ActiveActiveContext] that has the following methods: +The function passed to both factory methods return an `EventSourcedBehavior` and provide access to the @apidoc[ReplicationContext] that has the following methods: * entityId * replicaId @@ -108,8 +108,8 @@ As well as methods that **can only be** used in the event handler. The values th @@@ div { .group-java } -The function passed to both factory methods is invoked with a special @apidoc[ActiveActiveContext] that needs to be passed to the -concrete `ActiveActiveEventSourcedBehavior` and on to the super constructor. +The function passed to both factory methods is invoked with a special @apidoc[ReplicationContext] that needs to be passed to the +concrete `ReplicatedEventSourcedBehavior` and on to the super constructor. The context gives access to: @@ -118,7 +118,7 @@ The context gives access to: * allReplicas * persistenceId -As well as methods that **can only be** used in the event handler, accessed through `getActiveActiveContext`. The values these methods return relate to the event that is being processed. +As well as methods that **can only be** used in the event handler, accessed through `getReplicationContext`. The values these methods return relate to the event that is being processed. @@@ @@ -190,9 +190,9 @@ There is no built in support for knowing an event has been replicated to all rep For some use cases you may need to trigger side effects after consuming replicated events. For example when an auction has been closed in all data centers and all bids have been replicated. -The @api[ActiveActiveContext] contains the current replica, the origin replica for the event processes, and if a recovery is running. These can be used to +The @api[ReplicationContext] contains the current replica, the origin replica for the event processes, and if a recovery is running. These can be used to implement side effects that take place once events are fully replicated. If the side effect should happen only once then a particular replica can be -designated to do it. The @ref[Auction example](./persistence-active-active-examples.md#auction) uses these techniques. +designated to do it. The @ref[Auction example](./replicated-eventsourcing-examples.md#auction) uses these techniques. ## How it works @@ -203,7 +203,7 @@ You don’t have to read this section to be able to use the feature, but to use Causal delivery order means that events persisted in one data center are read in the same order in other data centers. The order of concurrent events is undefined, which should be no problem when using [CRDT's](#conflict-free-replicated-data-types) -and otherwise will be detected via the `ActiveActiveContext` concurrent method. +and otherwise will be detected via the `ReplicationContext` concurrent method. For example: @@ -231,7 +231,7 @@ A third data center may also see the events as either "e1, e3, e2" or as "e1, e2 ### Concurrent updates -Active-active automatically tracks causality between events from different replias using [version vectors](https://en.wikipedia.org/wiki/Version_vector). +Replicated Event Sourcing automatically tracks causality between events from different replicas using [version vectors](https://en.wikipedia.org/wiki/Version_vector). ![images/causality.png](images/causality.png) @@ -245,29 +245,29 @@ When comparing two version vectors `v1` and `v2`: * `v1`is CONCURRENT with `v2` otherwise -## Sharded Active Active entities +## Sharded Replicated Event Sourced entities -To simplify what probably are the most common use cases for how you will want to distribute the active active actors there is a minimal API for running multiple instances of @ref[Akka Cluster Sharding](cluster-sharding.md), +To simplify what probably are the most common use cases for how you will want to distribute the replicated actors there is a minimal API for running multiple instances of @ref[Akka Cluster Sharding](cluster-sharding.md), each instance holding the entities for a single replica. The distribution of the replicas can be controlled either through cluster roles or using the @ref[multi datacenter](cluster-dc.md) support in Akka Cluster. -The API consists of bootstrapping logic for starting the sharding instances through @apidoc[ActiveActiveShardingExtension] available from the +The API consists of bootstrapping logic for starting the sharding instances through @apidoc[ReplicatedShardingExtension] available from the `akka-cluster-sharding-typed` module. Scala -: @@snip [ActiveActiveShardingSpec.scala](/akka-cluster-sharding-typed/src/test/scala/akka/cluster/sharding/typed/ActiveActiveShardingSpec.scala) { #bootstrap } +: @@snip [ReplicatedShardingSpec.scala](/akka-cluster-sharding-typed/src/test/scala/akka/cluster/sharding/typed/ReplicatedShardingSpec.scala) { #bootstrap } Java -: @@snip [ActiveActiveShardingTest.java](/akka-cluster-sharding-typed/src/test/java/akka/cluster/sharding/typed/ActiveActiveShardingTest.java) { #bootstrap } +: @@snip [ReplicatedShardingTest.java](/akka-cluster-sharding-typed/src/test/java/akka/cluster/sharding/typed/ReplicatedShardingTest.java) { #bootstrap } -`init` returns an @apidoc[ActiveActiveSharding] instance which gives access to @apidoc[EntityRef]s for each of the replicas for arbitrary routing logic: +`init` returns an @apidoc[ReplicatedSharding] instance which gives access to @apidoc[EntityRef]s for each of the replicas for arbitrary routing logic: Scala -: @@snip [ActiveActiveShardingSpec.scala](/akka-cluster-sharding-typed/src/test/scala/akka/cluster/sharding/typed/ActiveActiveShardingSpec.scala) { #all-entity-refs } +: @@snip [ReplicatedShardingSpec.scala](/akka-cluster-sharding-typed/src/test/scala/akka/cluster/sharding/typed/ReplicatedShardingSpec.scala) { #all-entity-refs } Java -: @@snip [ActiveActiveShardingTest.java](/akka-cluster-sharding-typed/src/test/java/akka/cluster/sharding/typed/ActiveActiveShardingTest.java) { #all-entity-refs } +: @@snip [ReplicatedShardingTest.java](/akka-cluster-sharding-typed/src/test/java/akka/cluster/sharding/typed/ReplicatedShardingTest.java) { #all-entity-refs } More advanced routing among the replicas is currently left as an exercise for the reader (or may be covered in a future release [#29281](https://github.com/akka/akka/issues/29281), [#29319](https://github.com/akka/akka/issues/29319)). @@ -275,24 +275,24 @@ More advanced routing among the replicas is currently left as an exercise for th ## Direct Replication of Events Normally an event has to be written in the journal and then picked up by the trailing read journal in the other replicas. -As an optimization the active active events can be published across the Akka cluster to the replicas. The read side +As an optimization the replicated events can be published across the Akka cluster to the replicas. The read side query is still needed as delivery is not guaranteed, but can be configured to poll the database less often since most events will arrive at the replicas through the cluster. -To enable this feature you first need to enable event publishing on the @scala[`EventSourcedBehavior`]@java[`ActiveActiveEventSourcedBehavior`] with `withEventPublishing` -and then enable direct replication through `withDirectReplication()` on @apidoc[ActiveActiveShardingSettings] (if not using - active active sharding the replication can be run standalone by starting the @apidoc[ActiveActiveShardingDirectReplication] actor). +To enable this feature you first need to enable event publishing on the @scala[`EventSourcedBehavior`]@java[`ReplicatedEventSourcedBehavior`] with `withEventPublishing` +and then enable direct replication through `withDirectReplication()` on @apidoc[ReplicatedShardingSettings] (if not using + replicated sharding the replication can be run standalone by starting the @apidoc[ShardingDirectReplication] actor). The "event publishing" feature publishes each event to the local system event bus as a side effect after it has been written, -the `ActiveActiveShardingDirectReplication` actor subscribes to these events and forwards them to the replicas allowing them +the @apidoc[ShardingDirectReplication] actor subscribes to these events and forwards them to the replicas allowing them to fast forward the stream of events for the origin replica. (With additional potential future support in journals for fast forwarding [#29311](https://github.com/akka/akka/issues/29311)). ## Journal Support -For a journal plugin to support active active it needs to store and read metadata for each event if it is defined in the @apiref[PersistentRepr] +For a journal plugin to support replication it needs to store and read metadata for each event if it is defined in the @apiref[PersistentRepr] `metadata` field. To attach the metadata after writing it, `PersistentRepr.withMetadata` is used. The @apidoc[JournalSpec] in the Persistence TCK provides a capability flag `supportsMetadata` to toggle verification that metadata is handled correctly. -For a snapshot plugin to support active active it needs to store and read metadata for the snapshot if it is defined in the @apiref[akka.persistence.SnapshotMetadata] `metadata` field. +For a snapshot plugin to support replication it needs to store and read metadata for the snapshot if it is defined in the @apiref[akka.persistence.SnapshotMetadata] `metadata` field. To attach the metadata when reading the snapshot the `akka.persistence.SnapshotMetadata.apply` factory overload taking a `metadata` parameter is used. The @apidoc[SnapshotStoreSpec] in the Persistence TCK provides a capability flag `supportsMetadata` to toggle verification that metadata is handled correctly. \ No newline at end of file diff --git a/akka-persistence-tck/src/main/scala/akka/persistence/CapabilityFlags.scala b/akka-persistence-tck/src/main/scala/akka/persistence/CapabilityFlags.scala index fde55aa446..26b4fbc700 100644 --- a/akka-persistence-tck/src/main/scala/akka/persistence/CapabilityFlags.scala +++ b/akka-persistence-tck/src/main/scala/akka/persistence/CapabilityFlags.scala @@ -72,7 +72,7 @@ trait SnapshotStoreCapabilityFlags extends CapabilityFlags { /** * When `true` enables tests which check if the snapshot store properly stores and - * loads metadata (needed for Active Active) along with the snapshots + * loads metadata (needed for replication) along with the snapshots */ protected def supportsMetadata: CapabilityFlag } diff --git a/akka-persistence-typed-tests/src/test/java/akka/persistence/typed/ActiveActiveTest.java b/akka-persistence-typed-tests/src/test/java/akka/persistence/typed/ReplicatedEventSourcingTest.java similarity index 91% rename from akka-persistence-typed-tests/src/test/java/akka/persistence/typed/ActiveActiveTest.java rename to akka-persistence-typed-tests/src/test/java/akka/persistence/typed/ReplicatedEventSourcingTest.java index 38299e1530..a22d3bbb85 100644 --- a/akka-persistence-typed-tests/src/test/java/akka/persistence/typed/ActiveActiveTest.java +++ b/akka-persistence-typed-tests/src/test/java/akka/persistence/typed/ReplicatedEventSourcingTest.java @@ -11,7 +11,6 @@ import akka.actor.testkit.typed.javadsl.TestProbe; import akka.actor.typed.ActorRef; import akka.actor.typed.Behavior; import akka.persistence.testkit.PersistenceTestKitPlugin; -import akka.persistence.testkit.javadsl.PersistenceTestKit; import akka.persistence.testkit.query.javadsl.PersistenceTestKitReadJournal; import akka.persistence.typed.javadsl.*; import com.typesafe.config.ConfigFactory; @@ -25,10 +24,10 @@ import java.util.*; import static akka.Done.done; import static org.junit.Assert.assertEquals; -public class ActiveActiveTest extends JUnitSuite { +public class ReplicatedEventSourcingTest extends JUnitSuite { static final class TestBehavior - extends ActiveActiveEventSourcedBehavior> { + extends ReplicatedEventSourcedBehavior> { interface Command {} static final class GetState implements Command { @@ -81,7 +80,7 @@ public class ActiveActiveTest extends JUnitSuite { public static Behavior create( String entityId, ReplicaId replicaId, Set allReplicas) { - return ActiveActiveEventSourcing.withSharedJournal( + return ReplicatedEventSourcing.withSharedJournal( entityId, replicaId, allReplicas, @@ -89,8 +88,8 @@ public class ActiveActiveTest extends JUnitSuite { TestBehavior::new); } - private TestBehavior(ActiveActiveContext activeActiveContext) { - super(activeActiveContext); + private TestBehavior(ReplicationContext replicationContext) { + super(replicationContext); } @Override @@ -124,7 +123,7 @@ public class ActiveActiveTest extends JUnitSuite { (GetReplica cmd) -> Effect() .none() - .thenRun(() -> cmd.replyTo.tell(getActiveActiveContext().replicaId()))) + .thenRun(() -> cmd.replyTo.tell(getReplicationContext().replicaId()))) .onCommand(Stop.class, __ -> Effect().stop()) .build(); } @@ -153,9 +152,9 @@ public class ActiveActiveTest extends JUnitSuite { @Rule public final LogCapturing logCapturing = new LogCapturing(); - // minimal test, full coverage over in ActiveActiveSpec + // minimal test, full coverage over in ReplicatedEventSourcingSpec @Test - public void activeActiveReplicationTest() { + public void replicatedEventSourcingReplicationTest() { ReplicaId dcA = new ReplicaId("DC-A"); ReplicaId dcB = new ReplicaId("DC-B"); ReplicaId dcC = new ReplicaId("DC-C"); diff --git a/akka-persistence-typed-tests/src/test/java/jdocs/akka/persistence/typed/ActiveActiveCompileOnlyTest.java b/akka-persistence-typed-tests/src/test/java/jdocs/akka/persistence/typed/ReplicatedEventSourcingCompileOnlyTest.java similarity index 74% rename from akka-persistence-typed-tests/src/test/java/jdocs/akka/persistence/typed/ActiveActiveCompileOnlyTest.java rename to akka-persistence-typed-tests/src/test/java/jdocs/akka/persistence/typed/ReplicatedEventSourcingCompileOnlyTest.java index 4020954f9f..d9928f2bd6 100644 --- a/akka-persistence-typed-tests/src/test/java/jdocs/akka/persistence/typed/ActiveActiveCompileOnlyTest.java +++ b/akka-persistence-typed-tests/src/test/java/jdocs/akka/persistence/typed/ReplicatedEventSourcingCompileOnlyTest.java @@ -9,7 +9,7 @@ import akka.persistence.typed.javadsl.*; import java.util.*; -public class ActiveActiveCompileOnlyTest { +public class ReplicatedEventSourcingCompileOnlyTest { // dummy for docs example interface Command {} @@ -19,11 +19,11 @@ public class ActiveActiveCompileOnlyTest { interface State {} static // #factory - final class MyActiceActiveEventSourcedBehavior - extends ActiveActiveEventSourcedBehavior { + final class MyReplicatedEventSourcedBehavior + extends ReplicatedEventSourcedBehavior { - public MyActiceActiveEventSourcedBehavior(ActiveActiveContext activeActiveContext) { - super(activeActiveContext); + public MyReplicatedEventSourcedBehavior(ReplicationContext replicationContext) { + super(replicationContext); } // ... implementation of abstract methods ... // #factory @@ -58,12 +58,12 @@ public class ActiveActiveCompileOnlyTest { String queryPluginId = ""; // #factory-shared - ActiveActiveEventSourcing.withSharedJournal( + ReplicatedEventSourcing.withSharedJournal( "entityId", DCA, allReplicas, queryPluginId, - context -> new MyActiceActiveEventSourcedBehavior(context)); + context -> new MyReplicatedEventSourcedBehavior(context)); // #factory-shared // #factory @@ -74,11 +74,11 @@ public class ActiveActiveCompileOnlyTest { allReplicasAndQueryPlugins.put(DCB, "journalForDCB"); EventSourcedBehavior behavior = - ActiveActiveEventSourcing.create( + ReplicatedEventSourcing.create( "entityId", DCA, allReplicasAndQueryPlugins, - context -> new MyActiceActiveEventSourcedBehavior(context)); + context -> new MyReplicatedEventSourcedBehavior(context)); // #factory } } diff --git a/akka-persistence-typed-tests/src/test/scala/akka/persistence/typed/MultiJournalActiveActiveSpec.scala b/akka-persistence-typed-tests/src/test/scala/akka/persistence/typed/MultiJournalReplicationSpec.scala similarity index 91% rename from akka-persistence-typed-tests/src/test/scala/akka/persistence/typed/MultiJournalActiveActiveSpec.scala rename to akka-persistence-typed-tests/src/test/scala/akka/persistence/typed/MultiJournalReplicationSpec.scala index d4e0803997..8de42eacaf 100644 --- a/akka-persistence-typed-tests/src/test/scala/akka/persistence/typed/MultiJournalActiveActiveSpec.scala +++ b/akka-persistence-typed-tests/src/test/scala/akka/persistence/typed/MultiJournalReplicationSpec.scala @@ -14,7 +14,7 @@ import akka.actor.typed.Behavior import akka.persistence.query.PersistenceQuery import akka.persistence.query.scaladsl.CurrentEventsByPersistenceIdQuery import akka.persistence.testkit.PersistenceTestKitPlugin -import akka.persistence.typed.scaladsl.ActiveActiveEventSourcing +import akka.persistence.typed.scaladsl.ReplicatedEventSourcing import akka.persistence.typed.scaladsl.Effect import akka.persistence.typed.scaladsl.EventSourcedBehavior import akka.stream.scaladsl.Sink @@ -23,7 +23,7 @@ import com.typesafe.config.ConfigFactory import org.scalatest.concurrent.Eventually import org.scalatest.wordspec.AnyWordSpecLike -object MultiJournalActiveActiveSpec { +object MultiJournalReplicationSpec { object Actor { sealed trait Command @@ -32,7 +32,7 @@ object MultiJournalActiveActiveSpec { private val writeJournalPerReplica = Map("R1" -> "journal1.journal", "R2" -> "journal2.journal") def apply(entityId: String, replicaId: String): Behavior[Command] = { - ActiveActiveEventSourcing( + ReplicatedEventSourcing( entityId, ReplicaId(replicaId), Map(ReplicaId("R1") -> "journal1.query", ReplicaId("R2") -> "journal2.query"))( @@ -65,15 +65,15 @@ object MultiJournalActiveActiveSpec { } -class MultiJournalActiveActiveSpec - extends ScalaTestWithActorTestKit(MultiJournalActiveActiveSpec.separateJournalsConfig) +class MultiJournalReplicationSpec + extends ScalaTestWithActorTestKit(MultiJournalReplicationSpec.separateJournalsConfig) with AnyWordSpecLike with LogCapturing with Eventually { - import MultiJournalActiveActiveSpec._ + import MultiJournalReplicationSpec._ val ids = new AtomicInteger(0) def nextEntityId = s"e-${ids.getAndIncrement()}" - "ActiveActiveEventSourcing" should { + "ReplicatedEventSourcing" should { "support one journal per replica" in { val r1 = spawn(Actor("id1", "R1")) diff --git a/akka-persistence-typed-tests/src/test/scala/akka/persistence/typed/ActiveActiveEventPublishingSpec.scala b/akka-persistence-typed-tests/src/test/scala/akka/persistence/typed/ReplicatedEventPublishingSpec.scala similarity index 75% rename from akka-persistence-typed-tests/src/test/scala/akka/persistence/typed/ActiveActiveEventPublishingSpec.scala rename to akka-persistence-typed-tests/src/test/scala/akka/persistence/typed/ReplicatedEventPublishingSpec.scala index 56c8042ac8..90c498efb8 100644 --- a/akka-persistence-typed-tests/src/test/scala/akka/persistence/typed/ActiveActiveEventPublishingSpec.scala +++ b/akka-persistence-typed-tests/src/test/scala/akka/persistence/typed/ReplicatedEventPublishingSpec.scala @@ -13,14 +13,14 @@ import akka.actor.typed.scaladsl.Behaviors import akka.persistence.testkit.PersistenceTestKitPlugin import akka.persistence.testkit.query.scaladsl.PersistenceTestKitReadJournal import akka.persistence.typed.internal.{ ReplicatedPublishedEventMetaData, VersionVector } -import akka.persistence.typed.scaladsl.ActiveActiveEventSourcing +import akka.persistence.typed.scaladsl.ReplicatedEventSourcing import akka.persistence.typed.scaladsl.Effect import akka.persistence.typed.scaladsl.EventSourcedBehavior import org.scalatest.wordspec.AnyWordSpecLike -object ActiveActiveEventPublishingSpec { +object ReplicatedEventPublishingSpec { - object MyActiveActive { + object MyReplicatedBehavior { trait Command case class Add(text: String, replyTo: ActorRef[Done]) extends Command case class Get(replyTo: ActorRef[Set[String]]) extends Command @@ -28,7 +28,7 @@ object ActiveActiveEventPublishingSpec { def apply(entityId: String, replicaId: ReplicaId, allReplicas: Set[ReplicaId]): Behavior[Command] = Behaviors.setup { ctx => - ActiveActiveEventSourcing.withSharedJournal( + ReplicatedEventSourcing.withSharedJournal( entityId, replicaId, allReplicas, @@ -56,7 +56,7 @@ object ActiveActiveEventPublishingSpec { } } -class ActiveActiveEventPublishingSpec +class ReplicatedEventPublishingSpec extends ScalaTestWithActorTestKit(PersistenceTestKitPlugin.config) with AnyWordSpecLike with LogCapturing { @@ -71,14 +71,14 @@ class ActiveActiveEventPublishingSpec s"myId$idCounter" } - import ActiveActiveEventPublishingSpec._ + import ReplicatedEventPublishingSpec._ - "An active active actor" must { + "An Replicated Event Sourced actor" must { "move forward when a published event from a replica is received" in { val id = nextEntityId() - val actor = spawn(MyActiveActive(id, DCA, Set(DCA, DCB))) + val actor = spawn(MyReplicatedBehavior(id, DCA, Set(DCA, DCB))) val probe = createTestProbe[Any]() - actor ! MyActiveActive.Add("one", probe.ref) + actor ! MyReplicatedBehavior.Add("one", probe.ref) probe.expectMessage(Done) // simulate a published event from another replica @@ -88,18 +88,18 @@ class ActiveActiveEventPublishingSpec "two", System.currentTimeMillis(), Some(new ReplicatedPublishedEventMetaData(DCB, VersionVector.empty))) - actor ! MyActiveActive.Add("three", probe.ref) + actor ! MyReplicatedBehavior.Add("three", probe.ref) probe.expectMessage(Done) - actor ! MyActiveActive.Get(probe.ref) + actor ! MyReplicatedBehavior.Get(probe.ref) probe.expectMessage(Set("one", "two", "three")) } "ignore a published event from a replica is received but the sequence number is unexpected" in { val id = nextEntityId() - val actor = spawn(MyActiveActive(id, DCA, Set(DCA, DCB))) + val actor = spawn(MyReplicatedBehavior(id, DCA, Set(DCA, DCB))) val probe = createTestProbe[Any]() - actor ! MyActiveActive.Add("one", probe.ref) + actor ! MyReplicatedBehavior.Add("one", probe.ref) probe.expectMessage(Done) // simulate a published event from another replica @@ -109,18 +109,18 @@ class ActiveActiveEventPublishingSpec "two", System.currentTimeMillis(), Some(new ReplicatedPublishedEventMetaData(DCB, VersionVector.empty))) - actor ! MyActiveActive.Add("three", probe.ref) + actor ! MyReplicatedBehavior.Add("three", probe.ref) probe.expectMessage(Done) - actor ! MyActiveActive.Get(probe.ref) + actor ! MyReplicatedBehavior.Get(probe.ref) probe.expectMessage(Set("one", "three")) } "ignore a published event from an unknown replica" in { val id = nextEntityId() - val actor = spawn(MyActiveActive(id, DCA, Set(DCA, DCB))) + val actor = spawn(MyReplicatedBehavior(id, DCA, Set(DCA, DCB))) val probe = createTestProbe[Any]() - actor ! MyActiveActive.Add("one", probe.ref) + actor ! MyReplicatedBehavior.Add("one", probe.ref) probe.expectMessage(Done) // simulate a published event from another replica @@ -130,18 +130,18 @@ class ActiveActiveEventPublishingSpec "two", System.currentTimeMillis(), Some(new ReplicatedPublishedEventMetaData(DCC, VersionVector.empty))) - actor ! MyActiveActive.Add("three", probe.ref) + actor ! MyReplicatedBehavior.Add("three", probe.ref) probe.expectMessage(Done) - actor ! MyActiveActive.Get(probe.ref) + actor ! MyReplicatedBehavior.Get(probe.ref) probe.expectMessage(Set("one", "three")) } "ignore an already seen event from a replica" in { val id = nextEntityId() - val actor = spawn(MyActiveActive(id, DCA, Set(DCA, DCB))) + val actor = spawn(MyReplicatedBehavior(id, DCA, Set(DCA, DCB))) val probe = createTestProbe[Any]() - actor ! MyActiveActive.Add("one", probe.ref) + actor ! MyReplicatedBehavior.Add("one", probe.ref) probe.expectMessage(Done) // simulate a published event from another replica @@ -159,27 +159,27 @@ class ActiveActiveEventPublishingSpec System.currentTimeMillis(), Some(new ReplicatedPublishedEventMetaData(DCB, VersionVector.empty))) - actor ! MyActiveActive.Add("three", probe.ref) + actor ! MyReplicatedBehavior.Add("three", probe.ref) probe.expectMessage(Done) - actor ! MyActiveActive.Get(probe.ref) + actor ! MyReplicatedBehavior.Get(probe.ref) probe.expectMessage(Set("one", "two", "three")) } "handle published events after replay" in { val id = nextEntityId() val probe = createTestProbe[Any]() - val activeActiveBehavior = MyActiveActive(id, DCA, Set(DCA, DCB)) - val incarnation1 = spawn(activeActiveBehavior) - incarnation1 ! MyActiveActive.Add("one", probe.ref) + val replicatedBehavior = MyReplicatedBehavior(id, DCA, Set(DCA, DCB)) + val incarnation1 = spawn(replicatedBehavior) + incarnation1 ! MyReplicatedBehavior.Add("one", probe.ref) probe.expectMessage(Done) - incarnation1 ! MyActiveActive.Stop + incarnation1 ! MyReplicatedBehavior.Stop probe.expectTerminated(incarnation1) - val incarnation2 = spawn(activeActiveBehavior) + val incarnation2 = spawn(replicatedBehavior) - incarnation2 ! MyActiveActive.Get(probe.ref) + incarnation2 ! MyReplicatedBehavior.Get(probe.ref) probe.expectMessage(Set("one")) // replay completed @@ -191,19 +191,19 @@ class ActiveActiveEventPublishingSpec System.currentTimeMillis(), Some(new ReplicatedPublishedEventMetaData(DCB, VersionVector.empty))) - incarnation2 ! MyActiveActive.Add("three", probe.ref) + incarnation2 ! MyReplicatedBehavior.Add("three", probe.ref) probe.expectMessage(Done) - incarnation2 ! MyActiveActive.Get(probe.ref) + incarnation2 ! MyReplicatedBehavior.Get(probe.ref) probe.expectMessage(Set("one", "two", "three")) } "handle published events before and after replay" in { val id = nextEntityId() val probe = createTestProbe[Any]() - val activeActiveBehaviorA = MyActiveActive(id, DCA, Set(DCA, DCB)) - val incarnationA1 = spawn(activeActiveBehaviorA) - incarnationA1 ! MyActiveActive.Add("one", probe.ref) + val replicatedBehaviorA = MyReplicatedBehavior(id, DCA, Set(DCA, DCB)) + val incarnationA1 = spawn(replicatedBehaviorA) + incarnationA1 ! MyReplicatedBehavior.Add("one", probe.ref) probe.expectMessage(Done) // simulate a published event from another replica @@ -214,10 +214,10 @@ class ActiveActiveEventPublishingSpec System.currentTimeMillis(), Some(new ReplicatedPublishedEventMetaData(DCB, VersionVector.empty))) - incarnationA1 ! MyActiveActive.Stop + incarnationA1 ! MyReplicatedBehavior.Stop probe.expectTerminated(incarnationA1) - val incarnationA2 = spawn(activeActiveBehaviorA) + val incarnationA2 = spawn(replicatedBehaviorA) // simulate a published event from another replica incarnationA2.asInstanceOf[ActorRef[Any]] ! internal.PublishedEventImpl( @@ -227,10 +227,10 @@ class ActiveActiveEventPublishingSpec System.currentTimeMillis(), Some(new ReplicatedPublishedEventMetaData(DCB, VersionVector.empty))) - incarnationA2 ! MyActiveActive.Add("four", probe.ref) + incarnationA2 ! MyReplicatedBehavior.Add("four", probe.ref) probe.expectMessage(Done) - incarnationA2 ! MyActiveActive.Get(probe.ref) + incarnationA2 ! MyReplicatedBehavior.Get(probe.ref) probe.expectMessage(Set("one", "two", "three", "four")) } diff --git a/akka-persistence-typed-tests/src/test/scala/akka/persistence/typed/ActiveActiveSpec.scala b/akka-persistence-typed-tests/src/test/scala/akka/persistence/typed/ReplicatedEventSourcingSpec.scala similarity index 97% rename from akka-persistence-typed-tests/src/test/scala/akka/persistence/typed/ActiveActiveSpec.scala rename to akka-persistence-typed-tests/src/test/scala/akka/persistence/typed/ReplicatedEventSourcingSpec.scala index 3c6ea4e25b..d50c426a7a 100644 --- a/akka-persistence-typed-tests/src/test/scala/akka/persistence/typed/ActiveActiveSpec.scala +++ b/akka-persistence-typed-tests/src/test/scala/akka/persistence/typed/ReplicatedEventSourcingSpec.scala @@ -13,12 +13,12 @@ import akka.actor.typed.ActorRef import akka.actor.typed.Behavior import akka.persistence.testkit.PersistenceTestKitPlugin import akka.persistence.testkit.query.scaladsl.PersistenceTestKitReadJournal -import akka.persistence.typed.scaladsl.{ ActiveActiveContext, ActiveActiveEventSourcing, Effect, EventSourcedBehavior } +import akka.persistence.typed.scaladsl.{ Effect, EventSourcedBehavior, ReplicatedEventSourcing, ReplicationContext } import akka.serialization.jackson.CborSerializable import org.scalatest.concurrent.Eventually import org.scalatest.wordspec.AnyWordSpecLike -object ActiveActiveSpec { +object ReplicatedEventSourcingSpec { val AllReplicas = Set(ReplicaId("R1"), ReplicaId("R2"), ReplicaId("R3")) @@ -35,7 +35,7 @@ object ActiveActiveSpec { testBehavior(entityId, replicaId, Some(probe)) def eventSourcedBehavior( - aaContext: ActiveActiveContext, + aaContext: ReplicationContext, probe: Option[ActorRef[EventAndContext]]): EventSourcedBehavior[Command, String, State] = { EventSourcedBehavior[Command, String, State]( aaContext.persistenceId, @@ -65,7 +65,7 @@ object ActiveActiveSpec { entityId: String, replicaId: String, probe: Option[ActorRef[EventAndContext]] = None): Behavior[Command] = - ActiveActiveEventSourcing.withSharedJournal( + ReplicatedEventSourcing.withSharedJournal( entityId, ReplicaId(replicaId), AllReplicas, @@ -75,15 +75,15 @@ object ActiveActiveSpec { case class EventAndContext(event: Any, origin: ReplicaId, recoveryRunning: Boolean, concurrent: Boolean) -class ActiveActiveSpec +class ReplicatedEventSourcingSpec extends ScalaTestWithActorTestKit(PersistenceTestKitPlugin.config) with AnyWordSpecLike with LogCapturing with Eventually { - import ActiveActiveSpec._ + import ReplicatedEventSourcingSpec._ val ids = new AtomicInteger(0) def nextEntityId = s"e-${ids.getAndIncrement()}" - "ActiveActiveEventSourcing" should { + "ReplicatedEventSourcing" should { "replicate events between entities" in { val entityId = nextEntityId val probe = createTestProbe[Done]() diff --git a/akka-persistence-typed-tests/src/test/scala/akka/persistence/typed/ActiveActiveBaseSpec.scala b/akka-persistence-typed-tests/src/test/scala/akka/persistence/typed/ReplicationBaseSpec.scala similarity index 92% rename from akka-persistence-typed-tests/src/test/scala/akka/persistence/typed/ActiveActiveBaseSpec.scala rename to akka-persistence-typed-tests/src/test/scala/akka/persistence/typed/ReplicationBaseSpec.scala index cfa8a820dc..ec8a9487a7 100644 --- a/akka-persistence-typed-tests/src/test/scala/akka/persistence/typed/ActiveActiveBaseSpec.scala +++ b/akka-persistence-typed-tests/src/test/scala/akka/persistence/typed/ReplicationBaseSpec.scala @@ -11,13 +11,13 @@ import akka.persistence.testkit.{ PersistenceTestKitPlugin, PersistenceTestKitSn import org.scalatest.concurrent.Eventually import org.scalatest.wordspec.AnyWordSpecLike -object ActiveActiveBaseSpec { +object ReplicationBaseSpec { val R1 = ReplicaId("R1") val R2 = ReplicaId("R2") val AllReplicas = Set(R1, R2) } -abstract class ActiveActiveBaseSpec +abstract class ReplicationBaseSpec extends ScalaTestWithActorTestKit( PersistenceTestKitPlugin.config.withFallback(PersistenceTestKitSnapshotPlugin.config)) with AnyWordSpecLike diff --git a/akka-persistence-typed-tests/src/test/scala/akka/persistence/typed/ActiveActiveIllegalAccessSpec.scala b/akka-persistence-typed-tests/src/test/scala/akka/persistence/typed/ReplicationIllegalAccessSpec.scala similarity index 82% rename from akka-persistence-typed-tests/src/test/scala/akka/persistence/typed/ActiveActiveIllegalAccessSpec.scala rename to akka-persistence-typed-tests/src/test/scala/akka/persistence/typed/ReplicationIllegalAccessSpec.scala index a8145639b0..8415b0dcd9 100644 --- a/akka-persistence-typed-tests/src/test/scala/akka/persistence/typed/ActiveActiveIllegalAccessSpec.scala +++ b/akka-persistence-typed-tests/src/test/scala/akka/persistence/typed/ReplicationIllegalAccessSpec.scala @@ -8,12 +8,12 @@ import akka.actor.testkit.typed.scaladsl.{ LogCapturing, ScalaTestWithActorTestK import akka.actor.typed.{ ActorRef, Behavior } import akka.persistence.testkit.PersistenceTestKitPlugin import akka.persistence.testkit.query.scaladsl.PersistenceTestKitReadJournal -import akka.persistence.typed.scaladsl.{ ActiveActiveEventSourcing, Effect, EventSourcedBehavior } +import akka.persistence.typed.scaladsl.{ Effect, EventSourcedBehavior, ReplicatedEventSourcing } import akka.serialization.jackson.CborSerializable import org.scalatest.concurrent.Eventually import org.scalatest.wordspec.AnyWordSpecLike -object ActiveActiveIllegalAccessSpec { +object ReplicationIllegalAccessSpec { val R1 = ReplicaId("R1") val R2 = ReplicaId("R1") @@ -28,11 +28,7 @@ object ActiveActiveIllegalAccessSpec { case class State(all: List[String]) extends CborSerializable def apply(entityId: String, replica: ReplicaId): Behavior[Command] = { - ActiveActiveEventSourcing.withSharedJournal( - entityId, - replica, - AllReplicas, - PersistenceTestKitReadJournal.Identifier)( + ReplicatedEventSourcing.withSharedJournal(entityId, replica, AllReplicas, PersistenceTestKitReadJournal.Identifier)( aaContext => EventSourcedBehavior[Command, String, State]( aaContext.persistenceId, @@ -66,30 +62,30 @@ object ActiveActiveIllegalAccessSpec { } -class ActiveActiveIllegalAccessSpec +class ReplicationIllegalAccessSpec extends ScalaTestWithActorTestKit(PersistenceTestKitPlugin.config) with AnyWordSpecLike with LogCapturing with Eventually { - import ActiveActiveIllegalAccessSpec._ - "ActiveActive" should { + import ReplicationIllegalAccessSpec._ + "ReplicatedEventSourcing" should { "detect illegal access to context in command handler" in { val probe = createTestProbe[Thrown]() - val ref = spawn(ActiveActiveIllegalAccessSpec("id1", R1)) + val ref = spawn(ReplicationIllegalAccessSpec("id1", R1)) ref ! AccessInCommandHandler(probe.ref) val thrown: Throwable = probe.expectMessageType[Thrown].exception.get thrown.getMessage should include("from the event handler") } "detect illegal access to context in persist thenRun" in { val probe = createTestProbe[Thrown]() - val ref = spawn(ActiveActiveIllegalAccessSpec("id1", R1)) + val ref = spawn(ReplicationIllegalAccessSpec("id1", R1)) ref ! AccessInPersistCallback(probe.ref) val thrown: Throwable = probe.expectMessageType[Thrown].exception.get thrown.getMessage should include("from the event handler") } "detect illegal access in the factory" in { val exception = intercept[UnsupportedOperationException] { - ActiveActiveEventSourcing.withSharedJournal("id2", R1, AllReplicas, PersistenceTestKitReadJournal.Identifier) { + ReplicatedEventSourcing.withSharedJournal("id2", R1, AllReplicas, PersistenceTestKitReadJournal.Identifier) { aaContext => aaContext.origin ??? diff --git a/akka-persistence-typed-tests/src/test/scala/akka/persistence/typed/ActiveActiveSnapshotSpec.scala b/akka-persistence-typed-tests/src/test/scala/akka/persistence/typed/ReplicationSnapshotSpec.scala similarity index 92% rename from akka-persistence-typed-tests/src/test/scala/akka/persistence/typed/ActiveActiveSnapshotSpec.scala rename to akka-persistence-typed-tests/src/test/scala/akka/persistence/typed/ReplicationSnapshotSpec.scala index 8eeda55c15..f71b12afbb 100644 --- a/akka-persistence-typed-tests/src/test/scala/akka/persistence/typed/ActiveActiveSnapshotSpec.scala +++ b/akka-persistence-typed-tests/src/test/scala/akka/persistence/typed/ReplicationSnapshotSpec.scala @@ -12,13 +12,13 @@ import akka.persistence.testkit.{ PersistenceTestKitPlugin, PersistenceTestKitSn import akka.persistence.testkit.scaladsl.{ PersistenceTestKit, SnapshotTestKit } import akka.persistence.testkit.query.scaladsl.PersistenceTestKitReadJournal import akka.persistence.typed.internal.{ ReplicatedPublishedEventMetaData, VersionVector } -import akka.persistence.typed.scaladsl.ActiveActiveEventSourcing +import akka.persistence.typed.scaladsl.ReplicatedEventSourcing import org.scalatest.concurrent.Eventually import org.scalatest.wordspec.AnyWordSpecLike -object ActiveActiveSnapshotSpec { +object ReplicationSnapshotSpec { - import ActiveActiveSpec._ + import ReplicatedEventSourcingSpec._ def behaviorWithSnapshotting(entityId: String, replicaId: ReplicaId): Behavior[Command] = behaviorWithSnapshotting(entityId, replicaId, None) @@ -33,7 +33,7 @@ object ActiveActiveSnapshotSpec { entityId: String, replicaId: ReplicaId, probe: Option[ActorRef[EventAndContext]]): Behavior[Command] = { - ActiveActiveEventSourcing.withSharedJournal( + ReplicatedEventSourcing.withSharedJournal( entityId, replicaId, AllReplicas, @@ -43,14 +43,14 @@ object ActiveActiveSnapshotSpec { } } -class ActiveActiveSnapshotSpec +class ReplicationSnapshotSpec extends ScalaTestWithActorTestKit( PersistenceTestKitPlugin.config.withFallback(PersistenceTestKitSnapshotPlugin.config)) with AnyWordSpecLike with LogCapturing with Eventually { - import ActiveActiveSpec._ - import ActiveActiveSnapshotSpec._ + import ReplicatedEventSourcingSpec._ + import ReplicationSnapshotSpec._ val ids = new AtomicInteger(0) def nextEntityId = s"e-${ids.getAndIncrement()}" @@ -61,7 +61,7 @@ class ActiveActiveSnapshotSpec val R1 = ReplicaId("R1") val R2 = ReplicaId("R2") - "ActiveActive" should { + "ReplicatedEventSourcing" should { "recover state from snapshots" in { val entityId = nextEntityId val persistenceIdR1 = s"$entityId|R1" diff --git a/akka-persistence-typed-tests/src/test/scala/akka/persistence/typed/crdt/CounterSpec.scala b/akka-persistence-typed-tests/src/test/scala/akka/persistence/typed/crdt/CounterSpec.scala index 6393784eed..3f95c87755 100644 --- a/akka-persistence-typed-tests/src/test/scala/akka/persistence/typed/crdt/CounterSpec.scala +++ b/akka-persistence-typed-tests/src/test/scala/akka/persistence/typed/crdt/CounterSpec.scala @@ -8,8 +8,8 @@ import akka.actor.typed.ActorRef import akka.actor.typed.scaladsl.Behaviors import akka.persistence.testkit.query.scaladsl.PersistenceTestKitReadJournal import akka.persistence.typed.crdt.CounterSpec.PlainCounter.{ Decrement, Get, Increment } -import akka.persistence.typed.scaladsl.{ ActiveActiveEventSourcing, Effect, EventSourcedBehavior } -import akka.persistence.typed.{ ActiveActiveBaseSpec, ReplicaId } +import akka.persistence.typed.scaladsl.{ Effect, EventSourcedBehavior, ReplicatedEventSourcing } +import akka.persistence.typed.{ ReplicaId, ReplicationBaseSpec } object CounterSpec { @@ -20,7 +20,7 @@ object CounterSpec { case object Decrement extends Command } - import ActiveActiveBaseSpec._ + import ReplicationBaseSpec._ def apply( entityId: String, @@ -28,7 +28,7 @@ object CounterSpec { snapshotEvery: Long = 100, eventProbe: Option[ActorRef[Counter.Updated]] = None) = Behaviors.setup[PlainCounter.Command] { context => - ActiveActiveEventSourcing.withSharedJournal( + ReplicatedEventSourcing.withSharedJournal( entityId, replicaId, AllReplicas, @@ -58,12 +58,12 @@ object CounterSpec { } } -class CounterSpec extends ActiveActiveBaseSpec { +class CounterSpec extends ReplicationBaseSpec { import CounterSpec._ - import ActiveActiveBaseSpec._ + import ReplicationBaseSpec._ - "Active active entity using CRDT counter" should { + "Replicated entity using CRDT counter" should { "replicate" in { val id = nextEntityId val r1 = spawn(apply(id, R1)) diff --git a/akka-persistence-typed-tests/src/test/scala/akka/persistence/typed/crdt/LwwSpec.scala b/akka-persistence-typed-tests/src/test/scala/akka/persistence/typed/crdt/LwwSpec.scala index 8c0718b39a..48b300b5cf 100644 --- a/akka-persistence-typed-tests/src/test/scala/akka/persistence/typed/crdt/LwwSpec.scala +++ b/akka-persistence-typed-tests/src/test/scala/akka/persistence/typed/crdt/LwwSpec.scala @@ -6,13 +6,13 @@ package akka.persistence.typed.crdt import akka.actor.typed.{ ActorRef, Behavior } import akka.persistence.testkit.query.scaladsl.PersistenceTestKitReadJournal -import akka.persistence.typed.scaladsl.{ ActiveActiveEventSourcing, Effect, EventSourcedBehavior } -import akka.persistence.typed.{ ActiveActiveBaseSpec, ReplicaId } +import akka.persistence.typed.scaladsl.{ Effect, EventSourcedBehavior, ReplicatedEventSourcing } +import akka.persistence.typed.{ ReplicaId, ReplicationBaseSpec } import akka.serialization.jackson.CborSerializable object LwwSpec { - import ActiveActiveBaseSpec._ + import ReplicationBaseSpec._ sealed trait Command final case class Update(item: String, timestamp: Long, error: ActorRef[String]) extends Command @@ -26,7 +26,7 @@ object LwwSpec { object LwwRegistry { def apply(entityId: String, replica: ReplicaId): Behavior[Command] = { - ActiveActiveEventSourcing.withSharedJournal( + ReplicatedEventSourcing.withSharedJournal( entityId, replica, AllReplicas, @@ -59,9 +59,9 @@ object LwwSpec { } } -class LwwSpec extends ActiveActiveBaseSpec { +class LwwSpec extends ReplicationBaseSpec { import LwwSpec._ - import ActiveActiveBaseSpec._ + import ReplicationBaseSpec._ class Setup { val entityId = nextEntityId @@ -73,7 +73,7 @@ class LwwSpec extends ActiveActiveBaseSpec { val r2GetProbe = createTestProbe[Registry]() } - "Lww Active Active Event Sourced Behavior" should { + "Lww Replicated Event Sourced Behavior" should { "replicate a single event" in new Setup { r1 ! Update("a1", 1L, r1Probe.ref) eventually { diff --git a/akka-persistence-typed-tests/src/test/scala/akka/persistence/typed/crdt/ORSetSpec.scala b/akka-persistence-typed-tests/src/test/scala/akka/persistence/typed/crdt/ORSetSpec.scala index ce65f0b9da..e17afe3090 100644 --- a/akka-persistence-typed-tests/src/test/scala/akka/persistence/typed/crdt/ORSetSpec.scala +++ b/akka-persistence-typed-tests/src/test/scala/akka/persistence/typed/crdt/ORSetSpec.scala @@ -6,17 +6,17 @@ package akka.persistence.typed.crdt import akka.actor.typed.{ ActorRef, Behavior } import akka.persistence.testkit.query.scaladsl.PersistenceTestKitReadJournal -import akka.persistence.typed.scaladsl.{ ActiveActiveEventSourcing, Effect, EventSourcedBehavior } -import akka.persistence.typed.{ ActiveActiveBaseSpec, ReplicaId } +import akka.persistence.typed.scaladsl.{ Effect, EventSourcedBehavior, ReplicatedEventSourcing } +import akka.persistence.typed.{ ReplicaId, ReplicationBaseSpec } import ORSetSpec.ORSetEntity._ -import akka.persistence.typed.ActiveActiveBaseSpec.{ R1, R2 } +import akka.persistence.typed.ReplicationBaseSpec.{ R1, R2 } import akka.persistence.typed.crdt.ORSetSpec.ORSetEntity import scala.util.Random object ORSetSpec { - import ActiveActiveBaseSpec._ + import ReplicationBaseSpec._ object ORSetEntity { sealed trait Command @@ -27,7 +27,7 @@ object ORSetSpec { def apply(entityId: String, replica: ReplicaId): Behavior[ORSetEntity.Command] = { - ActiveActiveEventSourcing.withSharedJournal( + ReplicatedEventSourcing.withSharedJournal( entityId, replica, AllReplicas, @@ -54,7 +54,7 @@ object ORSetSpec { } -class ORSetSpec extends ActiveActiveBaseSpec { +class ORSetSpec extends ReplicationBaseSpec { class Setup { val entityId = nextEntityId @@ -78,7 +78,7 @@ class ORSetSpec extends ActiveActiveBaseSpec { Thread.sleep(Random.nextInt(200).toLong) } - "ORSet Active Active Entity" should { + "ORSet Replicated Entity" should { "support concurrent updates" in new Setup { r1 ! Add("a1") diff --git a/akka-persistence-typed-tests/src/test/scala/docs/akka/persistence/typed/AAAuctionExampleSpec.scala b/akka-persistence-typed-tests/src/test/scala/docs/akka/persistence/typed/ReplicatedAuctionExampleSpec.scala similarity index 95% rename from akka-persistence-typed-tests/src/test/scala/docs/akka/persistence/typed/AAAuctionExampleSpec.scala rename to akka-persistence-typed-tests/src/test/scala/docs/akka/persistence/typed/ReplicatedAuctionExampleSpec.scala index b0b553bc42..e0c4d408e2 100644 --- a/akka-persistence-typed-tests/src/test/scala/docs/akka/persistence/typed/AAAuctionExampleSpec.scala +++ b/akka-persistence-typed-tests/src/test/scala/docs/akka/persistence/typed/ReplicatedAuctionExampleSpec.scala @@ -12,13 +12,13 @@ import akka.actor.typed.{ ActorRef, Behavior } import akka.persistence.testkit.PersistenceTestKitPlugin import akka.persistence.testkit.query.scaladsl.PersistenceTestKitReadJournal import akka.persistence.typed.ReplicaId -import akka.persistence.typed.scaladsl.{ ActiveActiveContext, ActiveActiveEventSourcing, Effect, EventSourcedBehavior } +import akka.persistence.typed.scaladsl.{ Effect, EventSourcedBehavior, ReplicatedEventSourcing, ReplicationContext } import akka.serialization.jackson.CborSerializable import org.scalatest.concurrent.{ Eventually, ScalaFutures } import org.scalatest.matchers.should.Matchers import org.scalatest.wordspec.AnyWordSpecLike -object AAAuctionExampleSpec { +object ReplicatedAuctionExampleSpec { type MoneyAmount = Int @@ -104,7 +104,7 @@ object AAAuctionExampleSpec { //#setup //#command-handler - def commandHandler(setup: AuctionSetup, ctx: ActorContext[AuctionCommand], aaContext: ActiveActiveContext)( + def commandHandler(setup: AuctionSetup, ctx: ActorContext[AuctionCommand], aaContext: ReplicationContext)( state: AuctionState, command: AuctionCommand): Effect[AuctionEvent, AuctionState] = { state.phase match { @@ -166,7 +166,7 @@ object AAAuctionExampleSpec { } //#event-handler - def eventHandler(ctx: ActorContext[AuctionCommand], aaCtx: ActiveActiveContext, setup: AuctionSetup)( + def eventHandler(ctx: ActorContext[AuctionCommand], aaCtx: ReplicationContext, setup: AuctionSetup)( state: AuctionState, event: AuctionEvent): AuctionState = { @@ -184,7 +184,7 @@ object AAAuctionExampleSpec { private def eventTriggers( setup: AuctionSetup, ctx: ActorContext[AuctionCommand], - aaCtx: ActiveActiveContext, + aaCtx: ReplicationContext, event: AuctionEvent, newState: AuctionState) = { event match { @@ -214,7 +214,7 @@ object AAAuctionExampleSpec { def behavior(replica: ReplicaId, setup: AuctionSetup): Behavior[AuctionCommand] = Behaviors.setup[AuctionCommand] { ctx => - ActiveActiveEventSourcing + ReplicatedEventSourcing .withSharedJournal(setup.name, replica, setup.allReplicas, PersistenceTestKitReadJournal.Identifier) { aaCtx => EventSourcedBehavior( aaCtx.persistenceId, @@ -225,14 +225,14 @@ object AAAuctionExampleSpec { } } -class AAAuctionExampleSpec +class ReplicatedAuctionExampleSpec extends ScalaTestWithActorTestKit(PersistenceTestKitPlugin.config) with AnyWordSpecLike with Matchers with LogCapturing with ScalaFutures with Eventually { - import AAAuctionExampleSpec._ + import ReplicatedAuctionExampleSpec._ "Auction example" should { diff --git a/akka-persistence-typed-tests/src/test/scala/docs/akka/persistence/typed/AABlogExampleSpec.scala b/akka-persistence-typed-tests/src/test/scala/docs/akka/persistence/typed/ReplicatedBlogExampleSpec.scala similarity index 92% rename from akka-persistence-typed-tests/src/test/scala/docs/akka/persistence/typed/AABlogExampleSpec.scala rename to akka-persistence-typed-tests/src/test/scala/docs/akka/persistence/typed/ReplicatedBlogExampleSpec.scala index f67fd19226..2450592182 100644 --- a/akka-persistence-typed-tests/src/test/scala/docs/akka/persistence/typed/AABlogExampleSpec.scala +++ b/akka-persistence-typed-tests/src/test/scala/docs/akka/persistence/typed/ReplicatedBlogExampleSpec.scala @@ -19,7 +19,7 @@ import org.scalatest.matchers.should.Matchers import org.scalatest.time.{ Millis, Span } import org.scalatest.wordspec.AnyWordSpecLike -object AABlogExampleSpec { +object ReplicatedBlogExampleSpec { final case class BlogState(content: Option[PostContent], contentTimestamp: LwwTime, published: Boolean) { def withContent(newContent: PostContent, timestamp: LwwTime): BlogState = @@ -44,18 +44,18 @@ object AABlogExampleSpec { final case class BodyChanged(postId: String, newContent: PostContent, timestamp: LwwTime) extends BlogEvent } -class AABlogExampleSpec +class ReplicatedBlogExampleSpec extends ScalaTestWithActorTestKit(PersistenceTestKitPlugin.config) with AnyWordSpecLike with Matchers with LogCapturing with ScalaFutures with Eventually { - import AABlogExampleSpec._ + import ReplicatedBlogExampleSpec._ implicit val config: PatienceConfig = PatienceConfig(timeout = Span(timeout.duration.toMillis, Millis)) - def behavior(aa: ActiveActiveContext, ctx: ActorContext[BlogCommand]) = + def behavior(aa: ReplicationContext, ctx: ActorContext[BlogCommand]) = EventSourcedBehavior[BlogCommand, BlogEvent, BlogState]( aa.persistenceId, emptyState, @@ -114,11 +114,11 @@ class AABlogExampleSpec val refDcA: ActorRef[BlogCommand] = spawn( Behaviors.setup[BlogCommand] { ctx => - ActiveActiveEventSourcing.withSharedJournal( + ReplicatedEventSourcing.withSharedJournal( "cat", ReplicaId("DC-A"), Set(ReplicaId("DC-A"), ReplicaId("DC-B")), - PersistenceTestKitReadJournal.Identifier) { (aa: ActiveActiveContext) => + PersistenceTestKitReadJournal.Identifier) { (aa: ReplicationContext) => behavior(aa, ctx) } }, @@ -127,11 +127,11 @@ class AABlogExampleSpec val refDcB: ActorRef[BlogCommand] = spawn( Behaviors.setup[BlogCommand] { ctx => - ActiveActiveEventSourcing.withSharedJournal( + ReplicatedEventSourcing.withSharedJournal( "cat", ReplicaId("DC-B"), Set(ReplicaId("DC-A"), ReplicaId("DC-B")), - PersistenceTestKitReadJournal.Identifier) { (aa: ActiveActiveContext) => + PersistenceTestKitReadJournal.Identifier) { (aa: ReplicationContext) => behavior(aa, ctx) } }, diff --git a/akka-persistence-typed-tests/src/test/scala/docs/akka/persistence/typed/ActiveActiveCompileOnlySpec.scala b/akka-persistence-typed-tests/src/test/scala/docs/akka/persistence/typed/ReplicatedEventSourcingCompileOnlySpec.scala similarity index 64% rename from akka-persistence-typed-tests/src/test/scala/docs/akka/persistence/typed/ActiveActiveCompileOnlySpec.scala rename to akka-persistence-typed-tests/src/test/scala/docs/akka/persistence/typed/ReplicatedEventSourcingCompileOnlySpec.scala index dc0668ff6c..6d98e47bb9 100644 --- a/akka-persistence-typed-tests/src/test/scala/docs/akka/persistence/typed/ActiveActiveCompileOnlySpec.scala +++ b/akka-persistence-typed-tests/src/test/scala/docs/akka/persistence/typed/ReplicatedEventSourcingCompileOnlySpec.scala @@ -5,11 +5,11 @@ package docs.akka.persistence.typed import akka.persistence.typed.ReplicaId -import akka.persistence.typed.scaladsl.{ActiveActiveEventSourcing, EventSourcedBehavior} +import akka.persistence.typed.scaladsl.{ EventSourcedBehavior, ReplicatedEventSourcing } import com.github.ghik.silencer.silent @silent("never used") -object ActiveActiveCompileOnlySpec { +object ReplicatedEventSourcingCompileOnlySpec { //#replicas val DCA = ReplicaId("DC-A") @@ -24,13 +24,13 @@ object ActiveActiveCompileOnlySpec { trait Event //#factory-shared - ActiveActiveEventSourcing.withSharedJournal("entityId", DCA, AllReplicas, queryPluginId) { context => + ReplicatedEventSourcing.withSharedJournal("entityId", DCA, AllReplicas, queryPluginId) { context => EventSourcedBehavior[Command, State, Event](???, ???, ???, ???) } //#factory-shared //#factory - ActiveActiveEventSourcing("entityId", DCA, Map(DCA -> "journalForDCA", DCB -> "journalForDCB")) { context => + ReplicatedEventSourcing("entityId", DCA, Map(DCA -> "journalForDCA", DCB -> "journalForDCB")) { context => EventSourcedBehavior[Command, State, Event](???, ???, ???, ???) } //#factory diff --git a/akka-persistence-typed/src/main/java/akka/persistence/typed/serialization/ActiveActive.java b/akka-persistence-typed/src/main/java/akka/persistence/typed/serialization/ReplicatedEventSourcing.java similarity index 77% rename from akka-persistence-typed/src/main/java/akka/persistence/typed/serialization/ActiveActive.java rename to akka-persistence-typed/src/main/java/akka/persistence/typed/serialization/ReplicatedEventSourcing.java index cd0ca60693..2bc39615d6 100644 --- a/akka-persistence-typed/src/main/java/akka/persistence/typed/serialization/ActiveActive.java +++ b/akka-persistence-typed/src/main/java/akka/persistence/typed/serialization/ReplicatedEventSourcing.java @@ -3,12 +3,12 @@ */ // Generated by the protocol buffer compiler. DO NOT EDIT! -// source: ActiveActive.proto +// source: ReplicatedEventSourcing.proto package akka.persistence.typed.serialization; -public final class ActiveActive { - private ActiveActive() {} +public final class ReplicatedEventSourcing { + private ReplicatedEventSourcing() {} public static void registerAllExtensions( akka.protobufv3.internal.ExtensionRegistryLite registry) {} @@ -86,7 +86,7 @@ public final class ActiveActive { } public static final akka.protobufv3.internal.Descriptors.EnumDescriptor getDescriptor() { - return akka.persistence.typed.serialization.ActiveActive.getDescriptor() + return akka.persistence.typed.serialization.ReplicatedEventSourcing.getDescriptor() .getEnumTypes() .get(0); } @@ -201,17 +201,18 @@ public final class ActiveActive { } public static final akka.protobufv3.internal.Descriptors.Descriptor getDescriptor() { - return akka.persistence.typed.serialization.ActiveActive.internal_static_Counter_descriptor; + return akka.persistence.typed.serialization.ReplicatedEventSourcing + .internal_static_Counter_descriptor; } @java.lang.Override protected akka.protobufv3.internal.GeneratedMessageV3.FieldAccessorTable internalGetFieldAccessorTable() { - return akka.persistence.typed.serialization.ActiveActive + return akka.persistence.typed.serialization.ReplicatedEventSourcing .internal_static_Counter_fieldAccessorTable .ensureFieldAccessorsInitialized( - akka.persistence.typed.serialization.ActiveActive.Counter.class, - akka.persistence.typed.serialization.ActiveActive.Counter.Builder.class); + akka.persistence.typed.serialization.ReplicatedEventSourcing.Counter.class, + akka.persistence.typed.serialization.ReplicatedEventSourcing.Counter.Builder.class); } private int bitField0_; @@ -278,11 +279,11 @@ public final class ActiveActive { if (obj == this) { return true; } - if (!(obj instanceof akka.persistence.typed.serialization.ActiveActive.Counter)) { + if (!(obj instanceof akka.persistence.typed.serialization.ReplicatedEventSourcing.Counter)) { return super.equals(obj); } - akka.persistence.typed.serialization.ActiveActive.Counter other = - (akka.persistence.typed.serialization.ActiveActive.Counter) obj; + akka.persistence.typed.serialization.ReplicatedEventSourcing.Counter other = + (akka.persistence.typed.serialization.ReplicatedEventSourcing.Counter) obj; if (hasValue() != other.hasValue()) return false; if (hasValue()) { @@ -308,72 +309,74 @@ public final class ActiveActive { return hash; } - public static akka.persistence.typed.serialization.ActiveActive.Counter parseFrom( + public static akka.persistence.typed.serialization.ReplicatedEventSourcing.Counter parseFrom( java.nio.ByteBuffer data) throws akka.protobufv3.internal.InvalidProtocolBufferException { return PARSER.parseFrom(data); } - public static akka.persistence.typed.serialization.ActiveActive.Counter parseFrom( + public static akka.persistence.typed.serialization.ReplicatedEventSourcing.Counter parseFrom( java.nio.ByteBuffer data, akka.protobufv3.internal.ExtensionRegistryLite extensionRegistry) throws akka.protobufv3.internal.InvalidProtocolBufferException { return PARSER.parseFrom(data, extensionRegistry); } - public static akka.persistence.typed.serialization.ActiveActive.Counter parseFrom( + public static akka.persistence.typed.serialization.ReplicatedEventSourcing.Counter parseFrom( akka.protobufv3.internal.ByteString data) throws akka.protobufv3.internal.InvalidProtocolBufferException { return PARSER.parseFrom(data); } - public static akka.persistence.typed.serialization.ActiveActive.Counter parseFrom( + public static akka.persistence.typed.serialization.ReplicatedEventSourcing.Counter parseFrom( akka.protobufv3.internal.ByteString data, akka.protobufv3.internal.ExtensionRegistryLite extensionRegistry) throws akka.protobufv3.internal.InvalidProtocolBufferException { return PARSER.parseFrom(data, extensionRegistry); } - public static akka.persistence.typed.serialization.ActiveActive.Counter parseFrom(byte[] data) - throws akka.protobufv3.internal.InvalidProtocolBufferException { + public static akka.persistence.typed.serialization.ReplicatedEventSourcing.Counter parseFrom( + byte[] data) throws akka.protobufv3.internal.InvalidProtocolBufferException { return PARSER.parseFrom(data); } - public static akka.persistence.typed.serialization.ActiveActive.Counter parseFrom( + public static akka.persistence.typed.serialization.ReplicatedEventSourcing.Counter parseFrom( byte[] data, akka.protobufv3.internal.ExtensionRegistryLite extensionRegistry) throws akka.protobufv3.internal.InvalidProtocolBufferException { return PARSER.parseFrom(data, extensionRegistry); } - public static akka.persistence.typed.serialization.ActiveActive.Counter parseFrom( + public static akka.persistence.typed.serialization.ReplicatedEventSourcing.Counter parseFrom( java.io.InputStream input) throws java.io.IOException { return akka.protobufv3.internal.GeneratedMessageV3.parseWithIOException(PARSER, input); } - public static akka.persistence.typed.serialization.ActiveActive.Counter parseFrom( + public static akka.persistence.typed.serialization.ReplicatedEventSourcing.Counter parseFrom( java.io.InputStream input, akka.protobufv3.internal.ExtensionRegistryLite extensionRegistry) throws java.io.IOException { return akka.protobufv3.internal.GeneratedMessageV3.parseWithIOException( PARSER, input, extensionRegistry); } - public static akka.persistence.typed.serialization.ActiveActive.Counter parseDelimitedFrom( - java.io.InputStream input) throws java.io.IOException { + public static akka.persistence.typed.serialization.ReplicatedEventSourcing.Counter + parseDelimitedFrom(java.io.InputStream input) throws java.io.IOException { return akka.protobufv3.internal.GeneratedMessageV3.parseDelimitedWithIOException( PARSER, input); } - public static akka.persistence.typed.serialization.ActiveActive.Counter parseDelimitedFrom( - java.io.InputStream input, akka.protobufv3.internal.ExtensionRegistryLite extensionRegistry) - throws java.io.IOException { + public static akka.persistence.typed.serialization.ReplicatedEventSourcing.Counter + parseDelimitedFrom( + java.io.InputStream input, + akka.protobufv3.internal.ExtensionRegistryLite extensionRegistry) + throws java.io.IOException { return akka.protobufv3.internal.GeneratedMessageV3.parseDelimitedWithIOException( PARSER, input, extensionRegistry); } - public static akka.persistence.typed.serialization.ActiveActive.Counter parseFrom( + public static akka.persistence.typed.serialization.ReplicatedEventSourcing.Counter parseFrom( akka.protobufv3.internal.CodedInputStream input) throws java.io.IOException { return akka.protobufv3.internal.GeneratedMessageV3.parseWithIOException(PARSER, input); } - public static akka.persistence.typed.serialization.ActiveActive.Counter parseFrom( + public static akka.persistence.typed.serialization.ReplicatedEventSourcing.Counter parseFrom( akka.protobufv3.internal.CodedInputStream input, akka.protobufv3.internal.ExtensionRegistryLite extensionRegistry) throws java.io.IOException { @@ -391,7 +394,7 @@ public final class ActiveActive { } public static Builder newBuilder( - akka.persistence.typed.serialization.ActiveActive.Counter prototype) { + akka.persistence.typed.serialization.ReplicatedEventSourcing.Counter prototype) { return DEFAULT_INSTANCE.toBuilder().mergeFrom(prototype); } @@ -411,22 +414,24 @@ public final class ActiveActive { extends akka.protobufv3.internal.GeneratedMessageV3.Builder implements // @@protoc_insertion_point(builder_implements:Counter) - akka.persistence.typed.serialization.ActiveActive.CounterOrBuilder { + akka.persistence.typed.serialization.ReplicatedEventSourcing.CounterOrBuilder { public static final akka.protobufv3.internal.Descriptors.Descriptor getDescriptor() { - return akka.persistence.typed.serialization.ActiveActive.internal_static_Counter_descriptor; + return akka.persistence.typed.serialization.ReplicatedEventSourcing + .internal_static_Counter_descriptor; } @java.lang.Override protected akka.protobufv3.internal.GeneratedMessageV3.FieldAccessorTable internalGetFieldAccessorTable() { - return akka.persistence.typed.serialization.ActiveActive + return akka.persistence.typed.serialization.ReplicatedEventSourcing .internal_static_Counter_fieldAccessorTable .ensureFieldAccessorsInitialized( - akka.persistence.typed.serialization.ActiveActive.Counter.class, - akka.persistence.typed.serialization.ActiveActive.Counter.Builder.class); + akka.persistence.typed.serialization.ReplicatedEventSourcing.Counter.class, + akka.persistence.typed.serialization.ReplicatedEventSourcing.Counter.Builder.class); } - // Construct using akka.persistence.typed.serialization.ActiveActive.Counter.newBuilder() + // Construct using + // akka.persistence.typed.serialization.ReplicatedEventSourcing.Counter.newBuilder() private Builder() { maybeForceBuilderInitialization(); } @@ -450,17 +455,21 @@ public final class ActiveActive { @java.lang.Override public akka.protobufv3.internal.Descriptors.Descriptor getDescriptorForType() { - return akka.persistence.typed.serialization.ActiveActive.internal_static_Counter_descriptor; + return akka.persistence.typed.serialization.ReplicatedEventSourcing + .internal_static_Counter_descriptor; } @java.lang.Override - public akka.persistence.typed.serialization.ActiveActive.Counter getDefaultInstanceForType() { - return akka.persistence.typed.serialization.ActiveActive.Counter.getDefaultInstance(); + public akka.persistence.typed.serialization.ReplicatedEventSourcing.Counter + getDefaultInstanceForType() { + return akka.persistence.typed.serialization.ReplicatedEventSourcing.Counter + .getDefaultInstance(); } @java.lang.Override - public akka.persistence.typed.serialization.ActiveActive.Counter build() { - akka.persistence.typed.serialization.ActiveActive.Counter result = buildPartial(); + public akka.persistence.typed.serialization.ReplicatedEventSourcing.Counter build() { + akka.persistence.typed.serialization.ReplicatedEventSourcing.Counter result = + buildPartial(); if (!result.isInitialized()) { throw newUninitializedMessageException(result); } @@ -468,9 +477,9 @@ public final class ActiveActive { } @java.lang.Override - public akka.persistence.typed.serialization.ActiveActive.Counter buildPartial() { - akka.persistence.typed.serialization.ActiveActive.Counter result = - new akka.persistence.typed.serialization.ActiveActive.Counter(this); + public akka.persistence.typed.serialization.ReplicatedEventSourcing.Counter buildPartial() { + akka.persistence.typed.serialization.ReplicatedEventSourcing.Counter result = + new akka.persistence.typed.serialization.ReplicatedEventSourcing.Counter(this); int from_bitField0_ = bitField0_; int to_bitField0_ = 0; if (((from_bitField0_ & 0x00000001) != 0)) { @@ -519,17 +528,20 @@ public final class ActiveActive { @java.lang.Override public Builder mergeFrom(akka.protobufv3.internal.Message other) { - if (other instanceof akka.persistence.typed.serialization.ActiveActive.Counter) { - return mergeFrom((akka.persistence.typed.serialization.ActiveActive.Counter) other); + if (other instanceof akka.persistence.typed.serialization.ReplicatedEventSourcing.Counter) { + return mergeFrom( + (akka.persistence.typed.serialization.ReplicatedEventSourcing.Counter) other); } else { super.mergeFrom(other); return this; } } - public Builder mergeFrom(akka.persistence.typed.serialization.ActiveActive.Counter other) { - if (other == akka.persistence.typed.serialization.ActiveActive.Counter.getDefaultInstance()) - return this; + public Builder mergeFrom( + akka.persistence.typed.serialization.ReplicatedEventSourcing.Counter other) { + if (other + == akka.persistence.typed.serialization.ReplicatedEventSourcing.Counter + .getDefaultInstance()) return this; if (other.hasValue()) { setValue(other.getValue()); } @@ -551,12 +563,13 @@ public final class ActiveActive { akka.protobufv3.internal.CodedInputStream input, akka.protobufv3.internal.ExtensionRegistryLite extensionRegistry) throws java.io.IOException { - akka.persistence.typed.serialization.ActiveActive.Counter parsedMessage = null; + akka.persistence.typed.serialization.ReplicatedEventSourcing.Counter parsedMessage = null; try { parsedMessage = PARSER.parsePartialFrom(input, extensionRegistry); } catch (akka.protobufv3.internal.InvalidProtocolBufferException e) { parsedMessage = - (akka.persistence.typed.serialization.ActiveActive.Counter) e.getUnfinishedMessage(); + (akka.persistence.typed.serialization.ReplicatedEventSourcing.Counter) + e.getUnfinishedMessage(); throw e.unwrapIOException(); } finally { if (parsedMessage != null) { @@ -629,13 +642,15 @@ public final class ActiveActive { } // @@protoc_insertion_point(class_scope:Counter) - private static final akka.persistence.typed.serialization.ActiveActive.Counter DEFAULT_INSTANCE; + private static final akka.persistence.typed.serialization.ReplicatedEventSourcing.Counter + DEFAULT_INSTANCE; static { - DEFAULT_INSTANCE = new akka.persistence.typed.serialization.ActiveActive.Counter(); + DEFAULT_INSTANCE = new akka.persistence.typed.serialization.ReplicatedEventSourcing.Counter(); } - public static akka.persistence.typed.serialization.ActiveActive.Counter getDefaultInstance() { + public static akka.persistence.typed.serialization.ReplicatedEventSourcing.Counter + getDefaultInstance() { return DEFAULT_INSTANCE; } @@ -661,7 +676,8 @@ public final class ActiveActive { } @java.lang.Override - public akka.persistence.typed.serialization.ActiveActive.Counter getDefaultInstanceForType() { + public akka.persistence.typed.serialization.ReplicatedEventSourcing.Counter + getDefaultInstanceForType() { return DEFAULT_INSTANCE; } } @@ -757,18 +773,19 @@ public final class ActiveActive { } public static final akka.protobufv3.internal.Descriptors.Descriptor getDescriptor() { - return akka.persistence.typed.serialization.ActiveActive + return akka.persistence.typed.serialization.ReplicatedEventSourcing .internal_static_CounterUpdate_descriptor; } @java.lang.Override protected akka.protobufv3.internal.GeneratedMessageV3.FieldAccessorTable internalGetFieldAccessorTable() { - return akka.persistence.typed.serialization.ActiveActive + return akka.persistence.typed.serialization.ReplicatedEventSourcing .internal_static_CounterUpdate_fieldAccessorTable .ensureFieldAccessorsInitialized( - akka.persistence.typed.serialization.ActiveActive.CounterUpdate.class, - akka.persistence.typed.serialization.ActiveActive.CounterUpdate.Builder.class); + akka.persistence.typed.serialization.ReplicatedEventSourcing.CounterUpdate.class, + akka.persistence.typed.serialization.ReplicatedEventSourcing.CounterUpdate.Builder + .class); } private int bitField0_; @@ -835,11 +852,12 @@ public final class ActiveActive { if (obj == this) { return true; } - if (!(obj instanceof akka.persistence.typed.serialization.ActiveActive.CounterUpdate)) { + if (!(obj + instanceof akka.persistence.typed.serialization.ReplicatedEventSourcing.CounterUpdate)) { return super.equals(obj); } - akka.persistence.typed.serialization.ActiveActive.CounterUpdate other = - (akka.persistence.typed.serialization.ActiveActive.CounterUpdate) obj; + akka.persistence.typed.serialization.ReplicatedEventSourcing.CounterUpdate other = + (akka.persistence.typed.serialization.ReplicatedEventSourcing.CounterUpdate) obj; if (hasDelta() != other.hasDelta()) return false; if (hasDelta()) { @@ -865,60 +883,66 @@ public final class ActiveActive { return hash; } - public static akka.persistence.typed.serialization.ActiveActive.CounterUpdate parseFrom( - java.nio.ByteBuffer data) throws akka.protobufv3.internal.InvalidProtocolBufferException { + public static akka.persistence.typed.serialization.ReplicatedEventSourcing.CounterUpdate + parseFrom(java.nio.ByteBuffer data) + throws akka.protobufv3.internal.InvalidProtocolBufferException { return PARSER.parseFrom(data); } - public static akka.persistence.typed.serialization.ActiveActive.CounterUpdate parseFrom( - java.nio.ByteBuffer data, akka.protobufv3.internal.ExtensionRegistryLite extensionRegistry) - throws akka.protobufv3.internal.InvalidProtocolBufferException { + public static akka.persistence.typed.serialization.ReplicatedEventSourcing.CounterUpdate + parseFrom( + java.nio.ByteBuffer data, + akka.protobufv3.internal.ExtensionRegistryLite extensionRegistry) + throws akka.protobufv3.internal.InvalidProtocolBufferException { return PARSER.parseFrom(data, extensionRegistry); } - public static akka.persistence.typed.serialization.ActiveActive.CounterUpdate parseFrom( - akka.protobufv3.internal.ByteString data) - throws akka.protobufv3.internal.InvalidProtocolBufferException { + public static akka.persistence.typed.serialization.ReplicatedEventSourcing.CounterUpdate + parseFrom(akka.protobufv3.internal.ByteString data) + throws akka.protobufv3.internal.InvalidProtocolBufferException { return PARSER.parseFrom(data); } - public static akka.persistence.typed.serialization.ActiveActive.CounterUpdate parseFrom( - akka.protobufv3.internal.ByteString data, - akka.protobufv3.internal.ExtensionRegistryLite extensionRegistry) - throws akka.protobufv3.internal.InvalidProtocolBufferException { + public static akka.persistence.typed.serialization.ReplicatedEventSourcing.CounterUpdate + parseFrom( + akka.protobufv3.internal.ByteString data, + akka.protobufv3.internal.ExtensionRegistryLite extensionRegistry) + throws akka.protobufv3.internal.InvalidProtocolBufferException { return PARSER.parseFrom(data, extensionRegistry); } - public static akka.persistence.typed.serialization.ActiveActive.CounterUpdate parseFrom( - byte[] data) throws akka.protobufv3.internal.InvalidProtocolBufferException { + public static akka.persistence.typed.serialization.ReplicatedEventSourcing.CounterUpdate + parseFrom(byte[] data) throws akka.protobufv3.internal.InvalidProtocolBufferException { return PARSER.parseFrom(data); } - public static akka.persistence.typed.serialization.ActiveActive.CounterUpdate parseFrom( - byte[] data, akka.protobufv3.internal.ExtensionRegistryLite extensionRegistry) - throws akka.protobufv3.internal.InvalidProtocolBufferException { + public static akka.persistence.typed.serialization.ReplicatedEventSourcing.CounterUpdate + parseFrom(byte[] data, akka.protobufv3.internal.ExtensionRegistryLite extensionRegistry) + throws akka.protobufv3.internal.InvalidProtocolBufferException { return PARSER.parseFrom(data, extensionRegistry); } - public static akka.persistence.typed.serialization.ActiveActive.CounterUpdate parseFrom( - java.io.InputStream input) throws java.io.IOException { + public static akka.persistence.typed.serialization.ReplicatedEventSourcing.CounterUpdate + parseFrom(java.io.InputStream input) throws java.io.IOException { return akka.protobufv3.internal.GeneratedMessageV3.parseWithIOException(PARSER, input); } - public static akka.persistence.typed.serialization.ActiveActive.CounterUpdate parseFrom( - java.io.InputStream input, akka.protobufv3.internal.ExtensionRegistryLite extensionRegistry) - throws java.io.IOException { + public static akka.persistence.typed.serialization.ReplicatedEventSourcing.CounterUpdate + parseFrom( + java.io.InputStream input, + akka.protobufv3.internal.ExtensionRegistryLite extensionRegistry) + throws java.io.IOException { return akka.protobufv3.internal.GeneratedMessageV3.parseWithIOException( PARSER, input, extensionRegistry); } - public static akka.persistence.typed.serialization.ActiveActive.CounterUpdate + public static akka.persistence.typed.serialization.ReplicatedEventSourcing.CounterUpdate parseDelimitedFrom(java.io.InputStream input) throws java.io.IOException { return akka.protobufv3.internal.GeneratedMessageV3.parseDelimitedWithIOException( PARSER, input); } - public static akka.persistence.typed.serialization.ActiveActive.CounterUpdate + public static akka.persistence.typed.serialization.ReplicatedEventSourcing.CounterUpdate parseDelimitedFrom( java.io.InputStream input, akka.protobufv3.internal.ExtensionRegistryLite extensionRegistry) @@ -927,15 +951,16 @@ public final class ActiveActive { PARSER, input, extensionRegistry); } - public static akka.persistence.typed.serialization.ActiveActive.CounterUpdate parseFrom( - akka.protobufv3.internal.CodedInputStream input) throws java.io.IOException { + public static akka.persistence.typed.serialization.ReplicatedEventSourcing.CounterUpdate + parseFrom(akka.protobufv3.internal.CodedInputStream input) throws java.io.IOException { return akka.protobufv3.internal.GeneratedMessageV3.parseWithIOException(PARSER, input); } - public static akka.persistence.typed.serialization.ActiveActive.CounterUpdate parseFrom( - akka.protobufv3.internal.CodedInputStream input, - akka.protobufv3.internal.ExtensionRegistryLite extensionRegistry) - throws java.io.IOException { + public static akka.persistence.typed.serialization.ReplicatedEventSourcing.CounterUpdate + parseFrom( + akka.protobufv3.internal.CodedInputStream input, + akka.protobufv3.internal.ExtensionRegistryLite extensionRegistry) + throws java.io.IOException { return akka.protobufv3.internal.GeneratedMessageV3.parseWithIOException( PARSER, input, extensionRegistry); } @@ -950,7 +975,7 @@ public final class ActiveActive { } public static Builder newBuilder( - akka.persistence.typed.serialization.ActiveActive.CounterUpdate prototype) { + akka.persistence.typed.serialization.ReplicatedEventSourcing.CounterUpdate prototype) { return DEFAULT_INSTANCE.toBuilder().mergeFrom(prototype); } @@ -970,24 +995,25 @@ public final class ActiveActive { extends akka.protobufv3.internal.GeneratedMessageV3.Builder implements // @@protoc_insertion_point(builder_implements:CounterUpdate) - akka.persistence.typed.serialization.ActiveActive.CounterUpdateOrBuilder { + akka.persistence.typed.serialization.ReplicatedEventSourcing.CounterUpdateOrBuilder { public static final akka.protobufv3.internal.Descriptors.Descriptor getDescriptor() { - return akka.persistence.typed.serialization.ActiveActive + return akka.persistence.typed.serialization.ReplicatedEventSourcing .internal_static_CounterUpdate_descriptor; } @java.lang.Override protected akka.protobufv3.internal.GeneratedMessageV3.FieldAccessorTable internalGetFieldAccessorTable() { - return akka.persistence.typed.serialization.ActiveActive + return akka.persistence.typed.serialization.ReplicatedEventSourcing .internal_static_CounterUpdate_fieldAccessorTable .ensureFieldAccessorsInitialized( - akka.persistence.typed.serialization.ActiveActive.CounterUpdate.class, - akka.persistence.typed.serialization.ActiveActive.CounterUpdate.Builder.class); + akka.persistence.typed.serialization.ReplicatedEventSourcing.CounterUpdate.class, + akka.persistence.typed.serialization.ReplicatedEventSourcing.CounterUpdate.Builder + .class); } // Construct using - // akka.persistence.typed.serialization.ActiveActive.CounterUpdate.newBuilder() + // akka.persistence.typed.serialization.ReplicatedEventSourcing.CounterUpdate.newBuilder() private Builder() { maybeForceBuilderInitialization(); } @@ -1011,19 +1037,21 @@ public final class ActiveActive { @java.lang.Override public akka.protobufv3.internal.Descriptors.Descriptor getDescriptorForType() { - return akka.persistence.typed.serialization.ActiveActive + return akka.persistence.typed.serialization.ReplicatedEventSourcing .internal_static_CounterUpdate_descriptor; } @java.lang.Override - public akka.persistence.typed.serialization.ActiveActive.CounterUpdate + public akka.persistence.typed.serialization.ReplicatedEventSourcing.CounterUpdate getDefaultInstanceForType() { - return akka.persistence.typed.serialization.ActiveActive.CounterUpdate.getDefaultInstance(); + return akka.persistence.typed.serialization.ReplicatedEventSourcing.CounterUpdate + .getDefaultInstance(); } @java.lang.Override - public akka.persistence.typed.serialization.ActiveActive.CounterUpdate build() { - akka.persistence.typed.serialization.ActiveActive.CounterUpdate result = buildPartial(); + public akka.persistence.typed.serialization.ReplicatedEventSourcing.CounterUpdate build() { + akka.persistence.typed.serialization.ReplicatedEventSourcing.CounterUpdate result = + buildPartial(); if (!result.isInitialized()) { throw newUninitializedMessageException(result); } @@ -1031,9 +1059,10 @@ public final class ActiveActive { } @java.lang.Override - public akka.persistence.typed.serialization.ActiveActive.CounterUpdate buildPartial() { - akka.persistence.typed.serialization.ActiveActive.CounterUpdate result = - new akka.persistence.typed.serialization.ActiveActive.CounterUpdate(this); + public akka.persistence.typed.serialization.ReplicatedEventSourcing.CounterUpdate + buildPartial() { + akka.persistence.typed.serialization.ReplicatedEventSourcing.CounterUpdate result = + new akka.persistence.typed.serialization.ReplicatedEventSourcing.CounterUpdate(this); int from_bitField0_ = bitField0_; int to_bitField0_ = 0; if (((from_bitField0_ & 0x00000001) != 0)) { @@ -1082,8 +1111,10 @@ public final class ActiveActive { @java.lang.Override public Builder mergeFrom(akka.protobufv3.internal.Message other) { - if (other instanceof akka.persistence.typed.serialization.ActiveActive.CounterUpdate) { - return mergeFrom((akka.persistence.typed.serialization.ActiveActive.CounterUpdate) other); + if (other + instanceof akka.persistence.typed.serialization.ReplicatedEventSourcing.CounterUpdate) { + return mergeFrom( + (akka.persistence.typed.serialization.ReplicatedEventSourcing.CounterUpdate) other); } else { super.mergeFrom(other); return this; @@ -1091,10 +1122,10 @@ public final class ActiveActive { } public Builder mergeFrom( - akka.persistence.typed.serialization.ActiveActive.CounterUpdate other) { + akka.persistence.typed.serialization.ReplicatedEventSourcing.CounterUpdate other) { if (other - == akka.persistence.typed.serialization.ActiveActive.CounterUpdate.getDefaultInstance()) - return this; + == akka.persistence.typed.serialization.ReplicatedEventSourcing.CounterUpdate + .getDefaultInstance()) return this; if (other.hasDelta()) { setDelta(other.getDelta()); } @@ -1116,12 +1147,13 @@ public final class ActiveActive { akka.protobufv3.internal.CodedInputStream input, akka.protobufv3.internal.ExtensionRegistryLite extensionRegistry) throws java.io.IOException { - akka.persistence.typed.serialization.ActiveActive.CounterUpdate parsedMessage = null; + akka.persistence.typed.serialization.ReplicatedEventSourcing.CounterUpdate parsedMessage = + null; try { parsedMessage = PARSER.parsePartialFrom(input, extensionRegistry); } catch (akka.protobufv3.internal.InvalidProtocolBufferException e) { parsedMessage = - (akka.persistence.typed.serialization.ActiveActive.CounterUpdate) + (akka.persistence.typed.serialization.ReplicatedEventSourcing.CounterUpdate) e.getUnfinishedMessage(); throw e.unwrapIOException(); } finally { @@ -1195,14 +1227,15 @@ public final class ActiveActive { } // @@protoc_insertion_point(class_scope:CounterUpdate) - private static final akka.persistence.typed.serialization.ActiveActive.CounterUpdate + private static final akka.persistence.typed.serialization.ReplicatedEventSourcing.CounterUpdate DEFAULT_INSTANCE; static { - DEFAULT_INSTANCE = new akka.persistence.typed.serialization.ActiveActive.CounterUpdate(); + DEFAULT_INSTANCE = + new akka.persistence.typed.serialization.ReplicatedEventSourcing.CounterUpdate(); } - public static akka.persistence.typed.serialization.ActiveActive.CounterUpdate + public static akka.persistence.typed.serialization.ReplicatedEventSourcing.CounterUpdate getDefaultInstance() { return DEFAULT_INSTANCE; } @@ -1229,7 +1262,7 @@ public final class ActiveActive { } @java.lang.Override - public akka.persistence.typed.serialization.ActiveActive.CounterUpdate + public akka.persistence.typed.serialization.ReplicatedEventSourcing.CounterUpdate getDefaultInstanceForType() { return DEFAULT_INSTANCE; } @@ -1270,23 +1303,26 @@ public final class ActiveActive { * * @return The vvector. */ - akka.persistence.typed.serialization.ActiveActive.VersionVector getVvector(); + akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector getVvector(); /** required .VersionVector vvector = 2; */ - akka.persistence.typed.serialization.ActiveActive.VersionVectorOrBuilder getVvectorOrBuilder(); + akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVectorOrBuilder + getVvectorOrBuilder(); /** repeated .VersionVector dots = 3; */ - java.util.List getDotsList(); + java.util.List + getDotsList(); /** repeated .VersionVector dots = 3; */ - akka.persistence.typed.serialization.ActiveActive.VersionVector getDots(int index); + akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector getDots(int index); /** repeated .VersionVector dots = 3; */ int getDotsCount(); /** repeated .VersionVector dots = 3; */ java.util.List< - ? extends akka.persistence.typed.serialization.ActiveActive.VersionVectorOrBuilder> + ? extends + akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVectorOrBuilder> getDotsOrBuilderList(); /** repeated .VersionVector dots = 3; */ - akka.persistence.typed.serialization.ActiveActive.VersionVectorOrBuilder getDotsOrBuilder( - int index); + akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVectorOrBuilder + getDotsOrBuilder(int index); /** * repeated string stringElements = 4; @@ -1427,14 +1463,15 @@ public final class ActiveActive { } case 18: { - akka.persistence.typed.serialization.ActiveActive.VersionVector.Builder subBuilder = - null; + akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector.Builder + subBuilder = null; if (((bitField0_ & 0x00000002) != 0)) { subBuilder = vvector_.toBuilder(); } vvector_ = input.readMessage( - akka.persistence.typed.serialization.ActiveActive.VersionVector.PARSER, + akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector + .PARSER, extensionRegistry); if (subBuilder != null) { subBuilder.mergeFrom(vvector_); @@ -1448,12 +1485,14 @@ public final class ActiveActive { if (!((mutable_bitField0_ & 0x00000004) != 0)) { dots_ = new java.util.ArrayList< - akka.persistence.typed.serialization.ActiveActive.VersionVector>(); + akka.persistence.typed.serialization.ReplicatedEventSourcing + .VersionVector>(); mutable_bitField0_ |= 0x00000004; } dots_.add( input.readMessage( - akka.persistence.typed.serialization.ActiveActive.VersionVector.PARSER, + akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector + .PARSER, extensionRegistry)); break; } @@ -1560,17 +1599,18 @@ public final class ActiveActive { } public static final akka.protobufv3.internal.Descriptors.Descriptor getDescriptor() { - return akka.persistence.typed.serialization.ActiveActive.internal_static_ORSet_descriptor; + return akka.persistence.typed.serialization.ReplicatedEventSourcing + .internal_static_ORSet_descriptor; } @java.lang.Override protected akka.protobufv3.internal.GeneratedMessageV3.FieldAccessorTable internalGetFieldAccessorTable() { - return akka.persistence.typed.serialization.ActiveActive + return akka.persistence.typed.serialization.ReplicatedEventSourcing .internal_static_ORSet_fieldAccessorTable .ensureFieldAccessorsInitialized( - akka.persistence.typed.serialization.ActiveActive.ORSet.class, - akka.persistence.typed.serialization.ActiveActive.ORSet.Builder.class); + akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSet.class, + akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSet.Builder.class); } private int bitField0_; @@ -1620,7 +1660,7 @@ public final class ActiveActive { } public static final int VVECTOR_FIELD_NUMBER = 2; - private akka.persistence.typed.serialization.ActiveActive.VersionVector vvector_; + private akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector vvector_; /** * required .VersionVector vvector = 2; * @@ -1634,29 +1674,35 @@ public final class ActiveActive { * * @return The vvector. */ - public akka.persistence.typed.serialization.ActiveActive.VersionVector getVvector() { + public akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector getVvector() { return vvector_ == null - ? akka.persistence.typed.serialization.ActiveActive.VersionVector.getDefaultInstance() + ? akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector + .getDefaultInstance() : vvector_; } /** required .VersionVector vvector = 2; */ - public akka.persistence.typed.serialization.ActiveActive.VersionVectorOrBuilder + public akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVectorOrBuilder getVvectorOrBuilder() { return vvector_ == null - ? akka.persistence.typed.serialization.ActiveActive.VersionVector.getDefaultInstance() + ? akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector + .getDefaultInstance() : vvector_; } public static final int DOTS_FIELD_NUMBER = 3; - private java.util.List dots_; + private java.util.List< + akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector> + dots_; /** repeated .VersionVector dots = 3; */ - public java.util.List + public java.util.List< + akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector> getDotsList() { return dots_; } /** repeated .VersionVector dots = 3; */ public java.util.List< - ? extends akka.persistence.typed.serialization.ActiveActive.VersionVectorOrBuilder> + ? extends + akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVectorOrBuilder> getDotsOrBuilderList() { return dots_; } @@ -1665,11 +1711,12 @@ public final class ActiveActive { return dots_.size(); } /** repeated .VersionVector dots = 3; */ - public akka.persistence.typed.serialization.ActiveActive.VersionVector getDots(int index) { + public akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector getDots( + int index) { return dots_.get(index); } /** repeated .VersionVector dots = 3; */ - public akka.persistence.typed.serialization.ActiveActive.VersionVectorOrBuilder + public akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVectorOrBuilder getDotsOrBuilder(int index) { return dots_.get(index); } @@ -1933,11 +1980,11 @@ public final class ActiveActive { if (obj == this) { return true; } - if (!(obj instanceof akka.persistence.typed.serialization.ActiveActive.ORSet)) { + if (!(obj instanceof akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSet)) { return super.equals(obj); } - akka.persistence.typed.serialization.ActiveActive.ORSet other = - (akka.persistence.typed.serialization.ActiveActive.ORSet) obj; + akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSet other = + (akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSet) obj; if (hasOriginDc() != other.hasOriginDc()) return false; if (hasOriginDc()) { @@ -1996,72 +2043,74 @@ public final class ActiveActive { return hash; } - public static akka.persistence.typed.serialization.ActiveActive.ORSet parseFrom( + public static akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSet parseFrom( java.nio.ByteBuffer data) throws akka.protobufv3.internal.InvalidProtocolBufferException { return PARSER.parseFrom(data); } - public static akka.persistence.typed.serialization.ActiveActive.ORSet parseFrom( + public static akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSet parseFrom( java.nio.ByteBuffer data, akka.protobufv3.internal.ExtensionRegistryLite extensionRegistry) throws akka.protobufv3.internal.InvalidProtocolBufferException { return PARSER.parseFrom(data, extensionRegistry); } - public static akka.persistence.typed.serialization.ActiveActive.ORSet parseFrom( + public static akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSet parseFrom( akka.protobufv3.internal.ByteString data) throws akka.protobufv3.internal.InvalidProtocolBufferException { return PARSER.parseFrom(data); } - public static akka.persistence.typed.serialization.ActiveActive.ORSet parseFrom( + public static akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSet parseFrom( akka.protobufv3.internal.ByteString data, akka.protobufv3.internal.ExtensionRegistryLite extensionRegistry) throws akka.protobufv3.internal.InvalidProtocolBufferException { return PARSER.parseFrom(data, extensionRegistry); } - public static akka.persistence.typed.serialization.ActiveActive.ORSet parseFrom(byte[] data) - throws akka.protobufv3.internal.InvalidProtocolBufferException { + public static akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSet parseFrom( + byte[] data) throws akka.protobufv3.internal.InvalidProtocolBufferException { return PARSER.parseFrom(data); } - public static akka.persistence.typed.serialization.ActiveActive.ORSet parseFrom( + public static akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSet parseFrom( byte[] data, akka.protobufv3.internal.ExtensionRegistryLite extensionRegistry) throws akka.protobufv3.internal.InvalidProtocolBufferException { return PARSER.parseFrom(data, extensionRegistry); } - public static akka.persistence.typed.serialization.ActiveActive.ORSet parseFrom( + public static akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSet parseFrom( java.io.InputStream input) throws java.io.IOException { return akka.protobufv3.internal.GeneratedMessageV3.parseWithIOException(PARSER, input); } - public static akka.persistence.typed.serialization.ActiveActive.ORSet parseFrom( + public static akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSet parseFrom( java.io.InputStream input, akka.protobufv3.internal.ExtensionRegistryLite extensionRegistry) throws java.io.IOException { return akka.protobufv3.internal.GeneratedMessageV3.parseWithIOException( PARSER, input, extensionRegistry); } - public static akka.persistence.typed.serialization.ActiveActive.ORSet parseDelimitedFrom( - java.io.InputStream input) throws java.io.IOException { + public static akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSet + parseDelimitedFrom(java.io.InputStream input) throws java.io.IOException { return akka.protobufv3.internal.GeneratedMessageV3.parseDelimitedWithIOException( PARSER, input); } - public static akka.persistence.typed.serialization.ActiveActive.ORSet parseDelimitedFrom( - java.io.InputStream input, akka.protobufv3.internal.ExtensionRegistryLite extensionRegistry) - throws java.io.IOException { + public static akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSet + parseDelimitedFrom( + java.io.InputStream input, + akka.protobufv3.internal.ExtensionRegistryLite extensionRegistry) + throws java.io.IOException { return akka.protobufv3.internal.GeneratedMessageV3.parseDelimitedWithIOException( PARSER, input, extensionRegistry); } - public static akka.persistence.typed.serialization.ActiveActive.ORSet parseFrom( + public static akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSet parseFrom( akka.protobufv3.internal.CodedInputStream input) throws java.io.IOException { return akka.protobufv3.internal.GeneratedMessageV3.parseWithIOException(PARSER, input); } - public static akka.persistence.typed.serialization.ActiveActive.ORSet parseFrom( + public static akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSet parseFrom( akka.protobufv3.internal.CodedInputStream input, akka.protobufv3.internal.ExtensionRegistryLite extensionRegistry) throws java.io.IOException { @@ -2079,7 +2128,7 @@ public final class ActiveActive { } public static Builder newBuilder( - akka.persistence.typed.serialization.ActiveActive.ORSet prototype) { + akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSet prototype) { return DEFAULT_INSTANCE.toBuilder().mergeFrom(prototype); } @@ -2099,22 +2148,24 @@ public final class ActiveActive { extends akka.protobufv3.internal.GeneratedMessageV3.Builder implements // @@protoc_insertion_point(builder_implements:ORSet) - akka.persistence.typed.serialization.ActiveActive.ORSetOrBuilder { + akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSetOrBuilder { public static final akka.protobufv3.internal.Descriptors.Descriptor getDescriptor() { - return akka.persistence.typed.serialization.ActiveActive.internal_static_ORSet_descriptor; + return akka.persistence.typed.serialization.ReplicatedEventSourcing + .internal_static_ORSet_descriptor; } @java.lang.Override protected akka.protobufv3.internal.GeneratedMessageV3.FieldAccessorTable internalGetFieldAccessorTable() { - return akka.persistence.typed.serialization.ActiveActive + return akka.persistence.typed.serialization.ReplicatedEventSourcing .internal_static_ORSet_fieldAccessorTable .ensureFieldAccessorsInitialized( - akka.persistence.typed.serialization.ActiveActive.ORSet.class, - akka.persistence.typed.serialization.ActiveActive.ORSet.Builder.class); + akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSet.class, + akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSet.Builder.class); } - // Construct using akka.persistence.typed.serialization.ActiveActive.ORSet.newBuilder() + // Construct using + // akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSet.newBuilder() private Builder() { maybeForceBuilderInitialization(); } @@ -2166,17 +2217,20 @@ public final class ActiveActive { @java.lang.Override public akka.protobufv3.internal.Descriptors.Descriptor getDescriptorForType() { - return akka.persistence.typed.serialization.ActiveActive.internal_static_ORSet_descriptor; + return akka.persistence.typed.serialization.ReplicatedEventSourcing + .internal_static_ORSet_descriptor; } @java.lang.Override - public akka.persistence.typed.serialization.ActiveActive.ORSet getDefaultInstanceForType() { - return akka.persistence.typed.serialization.ActiveActive.ORSet.getDefaultInstance(); + public akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSet + getDefaultInstanceForType() { + return akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSet + .getDefaultInstance(); } @java.lang.Override - public akka.persistence.typed.serialization.ActiveActive.ORSet build() { - akka.persistence.typed.serialization.ActiveActive.ORSet result = buildPartial(); + public akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSet build() { + akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSet result = buildPartial(); if (!result.isInitialized()) { throw newUninitializedMessageException(result); } @@ -2184,9 +2238,9 @@ public final class ActiveActive { } @java.lang.Override - public akka.persistence.typed.serialization.ActiveActive.ORSet buildPartial() { - akka.persistence.typed.serialization.ActiveActive.ORSet result = - new akka.persistence.typed.serialization.ActiveActive.ORSet(this); + public akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSet buildPartial() { + akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSet result = + new akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSet(this); int from_bitField0_ = bitField0_; int to_bitField0_ = 0; if (((from_bitField0_ & 0x00000001) != 0)) { @@ -2276,17 +2330,20 @@ public final class ActiveActive { @java.lang.Override public Builder mergeFrom(akka.protobufv3.internal.Message other) { - if (other instanceof akka.persistence.typed.serialization.ActiveActive.ORSet) { - return mergeFrom((akka.persistence.typed.serialization.ActiveActive.ORSet) other); + if (other instanceof akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSet) { + return mergeFrom( + (akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSet) other); } else { super.mergeFrom(other); return this; } } - public Builder mergeFrom(akka.persistence.typed.serialization.ActiveActive.ORSet other) { - if (other == akka.persistence.typed.serialization.ActiveActive.ORSet.getDefaultInstance()) - return this; + public Builder mergeFrom( + akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSet other) { + if (other + == akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSet + .getDefaultInstance()) return this; if (other.hasOriginDc()) { bitField0_ |= 0x00000001; originDc_ = other.originDc_; @@ -2413,12 +2470,13 @@ public final class ActiveActive { akka.protobufv3.internal.CodedInputStream input, akka.protobufv3.internal.ExtensionRegistryLite extensionRegistry) throws java.io.IOException { - akka.persistence.typed.serialization.ActiveActive.ORSet parsedMessage = null; + akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSet parsedMessage = null; try { parsedMessage = PARSER.parsePartialFrom(input, extensionRegistry); } catch (akka.protobufv3.internal.InvalidProtocolBufferException e) { parsedMessage = - (akka.persistence.typed.serialization.ActiveActive.ORSet) e.getUnfinishedMessage(); + (akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSet) + e.getUnfinishedMessage(); throw e.unwrapIOException(); } finally { if (parsedMessage != null) { @@ -2515,11 +2573,11 @@ public final class ActiveActive { return this; } - private akka.persistence.typed.serialization.ActiveActive.VersionVector vvector_; + private akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector vvector_; private akka.protobufv3.internal.SingleFieldBuilderV3< - akka.persistence.typed.serialization.ActiveActive.VersionVector, - akka.persistence.typed.serialization.ActiveActive.VersionVector.Builder, - akka.persistence.typed.serialization.ActiveActive.VersionVectorOrBuilder> + akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector, + akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector.Builder, + akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVectorOrBuilder> vvectorBuilder_; /** * required .VersionVector vvector = 2; @@ -2534,10 +2592,12 @@ public final class ActiveActive { * * @return The vvector. */ - public akka.persistence.typed.serialization.ActiveActive.VersionVector getVvector() { + public akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector + getVvector() { if (vvectorBuilder_ == null) { return vvector_ == null - ? akka.persistence.typed.serialization.ActiveActive.VersionVector.getDefaultInstance() + ? akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector + .getDefaultInstance() : vvector_; } else { return vvectorBuilder_.getMessage(); @@ -2545,7 +2605,7 @@ public final class ActiveActive { } /** required .VersionVector vvector = 2; */ public Builder setVvector( - akka.persistence.typed.serialization.ActiveActive.VersionVector value) { + akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector value) { if (vvectorBuilder_ == null) { if (value == null) { throw new NullPointerException(); @@ -2560,7 +2620,8 @@ public final class ActiveActive { } /** required .VersionVector vvector = 2; */ public Builder setVvector( - akka.persistence.typed.serialization.ActiveActive.VersionVector.Builder builderForValue) { + akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector.Builder + builderForValue) { if (vvectorBuilder_ == null) { vvector_ = builderForValue.build(); onChanged(); @@ -2572,15 +2633,16 @@ public final class ActiveActive { } /** required .VersionVector vvector = 2; */ public Builder mergeVvector( - akka.persistence.typed.serialization.ActiveActive.VersionVector value) { + akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector value) { if (vvectorBuilder_ == null) { if (((bitField0_ & 0x00000002) != 0) && vvector_ != null && vvector_ - != akka.persistence.typed.serialization.ActiveActive.VersionVector + != akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector .getDefaultInstance()) { vvector_ = - akka.persistence.typed.serialization.ActiveActive.VersionVector.newBuilder(vvector_) + akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector + .newBuilder(vvector_) .mergeFrom(value) .buildPartial(); } else { @@ -2605,61 +2667,66 @@ public final class ActiveActive { return this; } /** required .VersionVector vvector = 2; */ - public akka.persistence.typed.serialization.ActiveActive.VersionVector.Builder + public akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector.Builder getVvectorBuilder() { bitField0_ |= 0x00000002; onChanged(); return getVvectorFieldBuilder().getBuilder(); } /** required .VersionVector vvector = 2; */ - public akka.persistence.typed.serialization.ActiveActive.VersionVectorOrBuilder + public akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVectorOrBuilder getVvectorOrBuilder() { if (vvectorBuilder_ != null) { return vvectorBuilder_.getMessageOrBuilder(); } else { return vvector_ == null - ? akka.persistence.typed.serialization.ActiveActive.VersionVector.getDefaultInstance() + ? akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector + .getDefaultInstance() : vvector_; } } /** required .VersionVector vvector = 2; */ private akka.protobufv3.internal.SingleFieldBuilderV3< - akka.persistence.typed.serialization.ActiveActive.VersionVector, - akka.persistence.typed.serialization.ActiveActive.VersionVector.Builder, - akka.persistence.typed.serialization.ActiveActive.VersionVectorOrBuilder> + akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector, + akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector.Builder, + akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVectorOrBuilder> getVvectorFieldBuilder() { if (vvectorBuilder_ == null) { vvectorBuilder_ = new akka.protobufv3.internal.SingleFieldBuilderV3< - akka.persistence.typed.serialization.ActiveActive.VersionVector, - akka.persistence.typed.serialization.ActiveActive.VersionVector.Builder, - akka.persistence.typed.serialization.ActiveActive.VersionVectorOrBuilder>( - getVvector(), getParentForChildren(), isClean()); + akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector, + akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector + .Builder, + akka.persistence.typed.serialization.ReplicatedEventSourcing + .VersionVectorOrBuilder>(getVvector(), getParentForChildren(), isClean()); vvector_ = null; } return vvectorBuilder_; } - private java.util.List + private java.util.List< + akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector> dots_ = java.util.Collections.emptyList(); private void ensureDotsIsMutable() { if (!((bitField0_ & 0x00000004) != 0)) { dots_ = new java.util.ArrayList< - akka.persistence.typed.serialization.ActiveActive.VersionVector>(dots_); + akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector>( + dots_); bitField0_ |= 0x00000004; } } private akka.protobufv3.internal.RepeatedFieldBuilderV3< - akka.persistence.typed.serialization.ActiveActive.VersionVector, - akka.persistence.typed.serialization.ActiveActive.VersionVector.Builder, - akka.persistence.typed.serialization.ActiveActive.VersionVectorOrBuilder> + akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector, + akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector.Builder, + akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVectorOrBuilder> dotsBuilder_; /** repeated .VersionVector dots = 3; */ - public java.util.List + public java.util.List< + akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector> getDotsList() { if (dotsBuilder_ == null) { return java.util.Collections.unmodifiableList(dots_); @@ -2676,7 +2743,8 @@ public final class ActiveActive { } } /** repeated .VersionVector dots = 3; */ - public akka.persistence.typed.serialization.ActiveActive.VersionVector getDots(int index) { + public akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector getDots( + int index) { if (dotsBuilder_ == null) { return dots_.get(index); } else { @@ -2685,7 +2753,8 @@ public final class ActiveActive { } /** repeated .VersionVector dots = 3; */ public Builder setDots( - int index, akka.persistence.typed.serialization.ActiveActive.VersionVector value) { + int index, + akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector value) { if (dotsBuilder_ == null) { if (value == null) { throw new NullPointerException(); @@ -2701,7 +2770,8 @@ public final class ActiveActive { /** repeated .VersionVector dots = 3; */ public Builder setDots( int index, - akka.persistence.typed.serialization.ActiveActive.VersionVector.Builder builderForValue) { + akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector.Builder + builderForValue) { if (dotsBuilder_ == null) { ensureDotsIsMutable(); dots_.set(index, builderForValue.build()); @@ -2713,7 +2783,7 @@ public final class ActiveActive { } /** repeated .VersionVector dots = 3; */ public Builder addDots( - akka.persistence.typed.serialization.ActiveActive.VersionVector value) { + akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector value) { if (dotsBuilder_ == null) { if (value == null) { throw new NullPointerException(); @@ -2728,7 +2798,8 @@ public final class ActiveActive { } /** repeated .VersionVector dots = 3; */ public Builder addDots( - int index, akka.persistence.typed.serialization.ActiveActive.VersionVector value) { + int index, + akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector value) { if (dotsBuilder_ == null) { if (value == null) { throw new NullPointerException(); @@ -2743,7 +2814,8 @@ public final class ActiveActive { } /** repeated .VersionVector dots = 3; */ public Builder addDots( - akka.persistence.typed.serialization.ActiveActive.VersionVector.Builder builderForValue) { + akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector.Builder + builderForValue) { if (dotsBuilder_ == null) { ensureDotsIsMutable(); dots_.add(builderForValue.build()); @@ -2756,7 +2828,8 @@ public final class ActiveActive { /** repeated .VersionVector dots = 3; */ public Builder addDots( int index, - akka.persistence.typed.serialization.ActiveActive.VersionVector.Builder builderForValue) { + akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector.Builder + builderForValue) { if (dotsBuilder_ == null) { ensureDotsIsMutable(); dots_.add(index, builderForValue.build()); @@ -2769,7 +2842,8 @@ public final class ActiveActive { /** repeated .VersionVector dots = 3; */ public Builder addAllDots( java.lang.Iterable< - ? extends akka.persistence.typed.serialization.ActiveActive.VersionVector> + ? extends + akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector> values) { if (dotsBuilder_ == null) { ensureDotsIsMutable(); @@ -2803,12 +2877,12 @@ public final class ActiveActive { return this; } /** repeated .VersionVector dots = 3; */ - public akka.persistence.typed.serialization.ActiveActive.VersionVector.Builder getDotsBuilder( - int index) { + public akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector.Builder + getDotsBuilder(int index) { return getDotsFieldBuilder().getBuilder(index); } /** repeated .VersionVector dots = 3; */ - public akka.persistence.typed.serialization.ActiveActive.VersionVectorOrBuilder + public akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVectorOrBuilder getDotsOrBuilder(int index) { if (dotsBuilder_ == null) { return dots_.get(index); @@ -2818,7 +2892,9 @@ public final class ActiveActive { } /** repeated .VersionVector dots = 3; */ public java.util.List< - ? extends akka.persistence.typed.serialization.ActiveActive.VersionVectorOrBuilder> + ? extends + akka.persistence.typed.serialization.ReplicatedEventSourcing + .VersionVectorOrBuilder> getDotsOrBuilderList() { if (dotsBuilder_ != null) { return dotsBuilder_.getMessageOrBuilderList(); @@ -2827,39 +2903,42 @@ public final class ActiveActive { } } /** repeated .VersionVector dots = 3; */ - public akka.persistence.typed.serialization.ActiveActive.VersionVector.Builder + public akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector.Builder addDotsBuilder() { return getDotsFieldBuilder() .addBuilder( - akka.persistence.typed.serialization.ActiveActive.VersionVector + akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector .getDefaultInstance()); } /** repeated .VersionVector dots = 3; */ - public akka.persistence.typed.serialization.ActiveActive.VersionVector.Builder addDotsBuilder( - int index) { + public akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector.Builder + addDotsBuilder(int index) { return getDotsFieldBuilder() .addBuilder( index, - akka.persistence.typed.serialization.ActiveActive.VersionVector + akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector .getDefaultInstance()); } /** repeated .VersionVector dots = 3; */ - public java.util.List + public java.util.List< + akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector.Builder> getDotsBuilderList() { return getDotsFieldBuilder().getBuilderList(); } private akka.protobufv3.internal.RepeatedFieldBuilderV3< - akka.persistence.typed.serialization.ActiveActive.VersionVector, - akka.persistence.typed.serialization.ActiveActive.VersionVector.Builder, - akka.persistence.typed.serialization.ActiveActive.VersionVectorOrBuilder> + akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector, + akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector.Builder, + akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVectorOrBuilder> getDotsFieldBuilder() { if (dotsBuilder_ == null) { dotsBuilder_ = new akka.protobufv3.internal.RepeatedFieldBuilderV3< - akka.persistence.typed.serialization.ActiveActive.VersionVector, - akka.persistence.typed.serialization.ActiveActive.VersionVector.Builder, - akka.persistence.typed.serialization.ActiveActive.VersionVectorOrBuilder>( + akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector, + akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector + .Builder, + akka.persistence.typed.serialization.ReplicatedEventSourcing + .VersionVectorOrBuilder>( dots_, ((bitField0_ & 0x00000004) != 0), getParentForChildren(), isClean()); dots_ = null; } @@ -3373,13 +3452,15 @@ public final class ActiveActive { } // @@protoc_insertion_point(class_scope:ORSet) - private static final akka.persistence.typed.serialization.ActiveActive.ORSet DEFAULT_INSTANCE; + private static final akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSet + DEFAULT_INSTANCE; static { - DEFAULT_INSTANCE = new akka.persistence.typed.serialization.ActiveActive.ORSet(); + DEFAULT_INSTANCE = new akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSet(); } - public static akka.persistence.typed.serialization.ActiveActive.ORSet getDefaultInstance() { + public static akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSet + getDefaultInstance() { return DEFAULT_INSTANCE; } @@ -3405,7 +3486,8 @@ public final class ActiveActive { } @java.lang.Override - public akka.persistence.typed.serialization.ActiveActive.ORSet getDefaultInstanceForType() { + public akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSet + getDefaultInstanceForType() { return DEFAULT_INSTANCE; } } @@ -3416,19 +3498,22 @@ public final class ActiveActive { akka.protobufv3.internal.MessageOrBuilder { /** repeated .ORSetDeltaGroup.Entry entries = 1; */ - java.util.List + java.util.List< + akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSetDeltaGroup.Entry> getEntriesList(); /** repeated .ORSetDeltaGroup.Entry entries = 1; */ - akka.persistence.typed.serialization.ActiveActive.ORSetDeltaGroup.Entry getEntries(int index); + akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSetDeltaGroup.Entry getEntries( + int index); /** repeated .ORSetDeltaGroup.Entry entries = 1; */ int getEntriesCount(); /** repeated .ORSetDeltaGroup.Entry entries = 1; */ java.util.List< ? extends - akka.persistence.typed.serialization.ActiveActive.ORSetDeltaGroup.EntryOrBuilder> + akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSetDeltaGroup + .EntryOrBuilder> getEntriesOrBuilderList(); /** repeated .ORSetDeltaGroup.Entry entries = 1; */ - akka.persistence.typed.serialization.ActiveActive.ORSetDeltaGroup.EntryOrBuilder + akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSetDeltaGroup.EntryOrBuilder getEntriesOrBuilder(int index); } /** Protobuf type {@code ORSetDeltaGroup} */ @@ -3482,14 +3567,14 @@ public final class ActiveActive { if (!((mutable_bitField0_ & 0x00000001) != 0)) { entries_ = new java.util.ArrayList< - akka.persistence.typed.serialization.ActiveActive.ORSetDeltaGroup - .Entry>(); + akka.persistence.typed.serialization.ReplicatedEventSourcing + .ORSetDeltaGroup.Entry>(); mutable_bitField0_ |= 0x00000001; } entries_.add( input.readMessage( - akka.persistence.typed.serialization.ActiveActive.ORSetDeltaGroup.Entry - .PARSER, + akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSetDeltaGroup + .Entry.PARSER, extensionRegistry)); break; } @@ -3517,18 +3602,19 @@ public final class ActiveActive { } public static final akka.protobufv3.internal.Descriptors.Descriptor getDescriptor() { - return akka.persistence.typed.serialization.ActiveActive + return akka.persistence.typed.serialization.ReplicatedEventSourcing .internal_static_ORSetDeltaGroup_descriptor; } @java.lang.Override protected akka.protobufv3.internal.GeneratedMessageV3.FieldAccessorTable internalGetFieldAccessorTable() { - return akka.persistence.typed.serialization.ActiveActive + return akka.persistence.typed.serialization.ReplicatedEventSourcing .internal_static_ORSetDeltaGroup_fieldAccessorTable .ensureFieldAccessorsInitialized( - akka.persistence.typed.serialization.ActiveActive.ORSetDeltaGroup.class, - akka.persistence.typed.serialization.ActiveActive.ORSetDeltaGroup.Builder.class); + akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSetDeltaGroup.class, + akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSetDeltaGroup.Builder + .class); } public interface EntryOrBuilder @@ -3547,7 +3633,7 @@ public final class ActiveActive { * * @return The operation. */ - akka.persistence.typed.serialization.ActiveActive.ORSetDeltaOp getOperation(); + akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSetDeltaOp getOperation(); /** * required .ORSet underlying = 2; @@ -3560,9 +3646,10 @@ public final class ActiveActive { * * @return The underlying. */ - akka.persistence.typed.serialization.ActiveActive.ORSet getUnderlying(); + akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSet getUnderlying(); /** required .ORSet underlying = 2; */ - akka.persistence.typed.serialization.ActiveActive.ORSetOrBuilder getUnderlyingOrBuilder(); + akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSetOrBuilder + getUnderlyingOrBuilder(); } /** Protobuf type {@code ORSetDeltaGroup.Entry} */ public static final class Entry extends akka.protobufv3.internal.GeneratedMessageV3 @@ -3614,9 +3701,9 @@ public final class ActiveActive { { int rawValue = input.readEnum(); @SuppressWarnings("deprecation") - akka.persistence.typed.serialization.ActiveActive.ORSetDeltaOp value = - akka.persistence.typed.serialization.ActiveActive.ORSetDeltaOp.valueOf( - rawValue); + akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSetDeltaOp value = + akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSetDeltaOp + .valueOf(rawValue); if (value == null) { unknownFields.mergeVarintField(1, rawValue); } else { @@ -3627,13 +3714,14 @@ public final class ActiveActive { } case 18: { - akka.persistence.typed.serialization.ActiveActive.ORSet.Builder subBuilder = null; + akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSet.Builder + subBuilder = null; if (((bitField0_ & 0x00000002) != 0)) { subBuilder = underlying_.toBuilder(); } underlying_ = input.readMessage( - akka.persistence.typed.serialization.ActiveActive.ORSet.PARSER, + akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSet.PARSER, extensionRegistry); if (subBuilder != null) { subBuilder.mergeFrom(underlying_); @@ -3663,19 +3751,20 @@ public final class ActiveActive { } public static final akka.protobufv3.internal.Descriptors.Descriptor getDescriptor() { - return akka.persistence.typed.serialization.ActiveActive + return akka.persistence.typed.serialization.ReplicatedEventSourcing .internal_static_ORSetDeltaGroup_Entry_descriptor; } @java.lang.Override protected akka.protobufv3.internal.GeneratedMessageV3.FieldAccessorTable internalGetFieldAccessorTable() { - return akka.persistence.typed.serialization.ActiveActive + return akka.persistence.typed.serialization.ReplicatedEventSourcing .internal_static_ORSetDeltaGroup_Entry_fieldAccessorTable .ensureFieldAccessorsInitialized( - akka.persistence.typed.serialization.ActiveActive.ORSetDeltaGroup.Entry.class, - akka.persistence.typed.serialization.ActiveActive.ORSetDeltaGroup.Entry.Builder - .class); + akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSetDeltaGroup.Entry + .class, + akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSetDeltaGroup.Entry + .Builder.class); } private int bitField0_; @@ -3694,17 +3783,19 @@ public final class ActiveActive { * * @return The operation. */ - public akka.persistence.typed.serialization.ActiveActive.ORSetDeltaOp getOperation() { + public akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSetDeltaOp + getOperation() { @SuppressWarnings("deprecation") - akka.persistence.typed.serialization.ActiveActive.ORSetDeltaOp result = - akka.persistence.typed.serialization.ActiveActive.ORSetDeltaOp.valueOf(operation_); + akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSetDeltaOp result = + akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSetDeltaOp.valueOf( + operation_); return result == null - ? akka.persistence.typed.serialization.ActiveActive.ORSetDeltaOp.Add + ? akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSetDeltaOp.Add : result; } public static final int UNDERLYING_FIELD_NUMBER = 2; - private akka.persistence.typed.serialization.ActiveActive.ORSet underlying_; + private akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSet underlying_; /** * required .ORSet underlying = 2; * @@ -3718,16 +3809,18 @@ public final class ActiveActive { * * @return The underlying. */ - public akka.persistence.typed.serialization.ActiveActive.ORSet getUnderlying() { + public akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSet getUnderlying() { return underlying_ == null - ? akka.persistence.typed.serialization.ActiveActive.ORSet.getDefaultInstance() + ? akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSet + .getDefaultInstance() : underlying_; } /** required .ORSet underlying = 2; */ - public akka.persistence.typed.serialization.ActiveActive.ORSetOrBuilder + public akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSetOrBuilder getUnderlyingOrBuilder() { return underlying_ == null - ? akka.persistence.typed.serialization.ActiveActive.ORSet.getDefaultInstance() + ? akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSet + .getDefaultInstance() : underlying_; } @@ -3790,11 +3883,13 @@ public final class ActiveActive { return true; } if (!(obj - instanceof akka.persistence.typed.serialization.ActiveActive.ORSetDeltaGroup.Entry)) { + instanceof + akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSetDeltaGroup.Entry)) { return super.equals(obj); } - akka.persistence.typed.serialization.ActiveActive.ORSetDeltaGroup.Entry other = - (akka.persistence.typed.serialization.ActiveActive.ORSetDeltaGroup.Entry) obj; + akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSetDeltaGroup.Entry other = + (akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSetDeltaGroup.Entry) + obj; if (hasOperation() != other.hasOperation()) return false; if (hasOperation()) { @@ -3828,13 +3923,15 @@ public final class ActiveActive { return hash; } - public static akka.persistence.typed.serialization.ActiveActive.ORSetDeltaGroup.Entry + public static akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSetDeltaGroup + .Entry parseFrom(java.nio.ByteBuffer data) throws akka.protobufv3.internal.InvalidProtocolBufferException { return PARSER.parseFrom(data); } - public static akka.persistence.typed.serialization.ActiveActive.ORSetDeltaGroup.Entry + public static akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSetDeltaGroup + .Entry parseFrom( java.nio.ByteBuffer data, akka.protobufv3.internal.ExtensionRegistryLite extensionRegistry) @@ -3842,13 +3939,15 @@ public final class ActiveActive { return PARSER.parseFrom(data, extensionRegistry); } - public static akka.persistence.typed.serialization.ActiveActive.ORSetDeltaGroup.Entry + public static akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSetDeltaGroup + .Entry parseFrom(akka.protobufv3.internal.ByteString data) throws akka.protobufv3.internal.InvalidProtocolBufferException { return PARSER.parseFrom(data); } - public static akka.persistence.typed.serialization.ActiveActive.ORSetDeltaGroup.Entry + public static akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSetDeltaGroup + .Entry parseFrom( akka.protobufv3.internal.ByteString data, akka.protobufv3.internal.ExtensionRegistryLite extensionRegistry) @@ -3856,23 +3955,27 @@ public final class ActiveActive { return PARSER.parseFrom(data, extensionRegistry); } - public static akka.persistence.typed.serialization.ActiveActive.ORSetDeltaGroup.Entry + public static akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSetDeltaGroup + .Entry parseFrom(byte[] data) throws akka.protobufv3.internal.InvalidProtocolBufferException { return PARSER.parseFrom(data); } - public static akka.persistence.typed.serialization.ActiveActive.ORSetDeltaGroup.Entry + public static akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSetDeltaGroup + .Entry parseFrom(byte[] data, akka.protobufv3.internal.ExtensionRegistryLite extensionRegistry) throws akka.protobufv3.internal.InvalidProtocolBufferException { return PARSER.parseFrom(data, extensionRegistry); } - public static akka.persistence.typed.serialization.ActiveActive.ORSetDeltaGroup.Entry + public static akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSetDeltaGroup + .Entry parseFrom(java.io.InputStream input) throws java.io.IOException { return akka.protobufv3.internal.GeneratedMessageV3.parseWithIOException(PARSER, input); } - public static akka.persistence.typed.serialization.ActiveActive.ORSetDeltaGroup.Entry + public static akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSetDeltaGroup + .Entry parseFrom( java.io.InputStream input, akka.protobufv3.internal.ExtensionRegistryLite extensionRegistry) @@ -3881,13 +3984,15 @@ public final class ActiveActive { PARSER, input, extensionRegistry); } - public static akka.persistence.typed.serialization.ActiveActive.ORSetDeltaGroup.Entry + public static akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSetDeltaGroup + .Entry parseDelimitedFrom(java.io.InputStream input) throws java.io.IOException { return akka.protobufv3.internal.GeneratedMessageV3.parseDelimitedWithIOException( PARSER, input); } - public static akka.persistence.typed.serialization.ActiveActive.ORSetDeltaGroup.Entry + public static akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSetDeltaGroup + .Entry parseDelimitedFrom( java.io.InputStream input, akka.protobufv3.internal.ExtensionRegistryLite extensionRegistry) @@ -3896,12 +4001,14 @@ public final class ActiveActive { PARSER, input, extensionRegistry); } - public static akka.persistence.typed.serialization.ActiveActive.ORSetDeltaGroup.Entry + public static akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSetDeltaGroup + .Entry parseFrom(akka.protobufv3.internal.CodedInputStream input) throws java.io.IOException { return akka.protobufv3.internal.GeneratedMessageV3.parseWithIOException(PARSER, input); } - public static akka.persistence.typed.serialization.ActiveActive.ORSetDeltaGroup.Entry + public static akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSetDeltaGroup + .Entry parseFrom( akka.protobufv3.internal.CodedInputStream input, akka.protobufv3.internal.ExtensionRegistryLite extensionRegistry) @@ -3920,7 +4027,8 @@ public final class ActiveActive { } public static Builder newBuilder( - akka.persistence.typed.serialization.ActiveActive.ORSetDeltaGroup.Entry prototype) { + akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSetDeltaGroup.Entry + prototype) { return DEFAULT_INSTANCE.toBuilder().mergeFrom(prototype); } @@ -3940,25 +4048,27 @@ public final class ActiveActive { extends akka.protobufv3.internal.GeneratedMessageV3.Builder implements // @@protoc_insertion_point(builder_implements:ORSetDeltaGroup.Entry) - akka.persistence.typed.serialization.ActiveActive.ORSetDeltaGroup.EntryOrBuilder { + akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSetDeltaGroup + .EntryOrBuilder { public static final akka.protobufv3.internal.Descriptors.Descriptor getDescriptor() { - return akka.persistence.typed.serialization.ActiveActive + return akka.persistence.typed.serialization.ReplicatedEventSourcing .internal_static_ORSetDeltaGroup_Entry_descriptor; } @java.lang.Override protected akka.protobufv3.internal.GeneratedMessageV3.FieldAccessorTable internalGetFieldAccessorTable() { - return akka.persistence.typed.serialization.ActiveActive + return akka.persistence.typed.serialization.ReplicatedEventSourcing .internal_static_ORSetDeltaGroup_Entry_fieldAccessorTable .ensureFieldAccessorsInitialized( - akka.persistence.typed.serialization.ActiveActive.ORSetDeltaGroup.Entry.class, - akka.persistence.typed.serialization.ActiveActive.ORSetDeltaGroup.Entry.Builder - .class); + akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSetDeltaGroup.Entry + .class, + akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSetDeltaGroup.Entry + .Builder.class); } // Construct using - // akka.persistence.typed.serialization.ActiveActive.ORSetDeltaGroup.Entry.newBuilder() + // akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSetDeltaGroup.Entry.newBuilder() private Builder() { maybeForceBuilderInitialization(); } @@ -3990,21 +4100,22 @@ public final class ActiveActive { @java.lang.Override public akka.protobufv3.internal.Descriptors.Descriptor getDescriptorForType() { - return akka.persistence.typed.serialization.ActiveActive + return akka.persistence.typed.serialization.ReplicatedEventSourcing .internal_static_ORSetDeltaGroup_Entry_descriptor; } @java.lang.Override - public akka.persistence.typed.serialization.ActiveActive.ORSetDeltaGroup.Entry + public akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSetDeltaGroup.Entry getDefaultInstanceForType() { - return akka.persistence.typed.serialization.ActiveActive.ORSetDeltaGroup.Entry + return akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSetDeltaGroup.Entry .getDefaultInstance(); } @java.lang.Override - public akka.persistence.typed.serialization.ActiveActive.ORSetDeltaGroup.Entry build() { - akka.persistence.typed.serialization.ActiveActive.ORSetDeltaGroup.Entry result = - buildPartial(); + public akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSetDeltaGroup.Entry + build() { + akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSetDeltaGroup.Entry + result = buildPartial(); if (!result.isInitialized()) { throw newUninitializedMessageException(result); } @@ -4012,10 +4123,12 @@ public final class ActiveActive { } @java.lang.Override - public akka.persistence.typed.serialization.ActiveActive.ORSetDeltaGroup.Entry + public akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSetDeltaGroup.Entry buildPartial() { - akka.persistence.typed.serialization.ActiveActive.ORSetDeltaGroup.Entry result = - new akka.persistence.typed.serialization.ActiveActive.ORSetDeltaGroup.Entry(this); + akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSetDeltaGroup.Entry + result = + new akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSetDeltaGroup + .Entry(this); int from_bitField0_ = bitField0_; int to_bitField0_ = 0; if (((from_bitField0_ & 0x00000001) != 0)) { @@ -4073,9 +4186,11 @@ public final class ActiveActive { @java.lang.Override public Builder mergeFrom(akka.protobufv3.internal.Message other) { if (other - instanceof akka.persistence.typed.serialization.ActiveActive.ORSetDeltaGroup.Entry) { + instanceof + akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSetDeltaGroup.Entry) { return mergeFrom( - (akka.persistence.typed.serialization.ActiveActive.ORSetDeltaGroup.Entry) other); + (akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSetDeltaGroup.Entry) + other); } else { super.mergeFrom(other); return this; @@ -4083,9 +4198,10 @@ public final class ActiveActive { } public Builder mergeFrom( - akka.persistence.typed.serialization.ActiveActive.ORSetDeltaGroup.Entry other) { + akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSetDeltaGroup.Entry + other) { if (other - == akka.persistence.typed.serialization.ActiveActive.ORSetDeltaGroup.Entry + == akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSetDeltaGroup.Entry .getDefaultInstance()) return this; if (other.hasOperation()) { setOperation(other.getOperation()); @@ -4117,13 +4233,13 @@ public final class ActiveActive { akka.protobufv3.internal.CodedInputStream input, akka.protobufv3.internal.ExtensionRegistryLite extensionRegistry) throws java.io.IOException { - akka.persistence.typed.serialization.ActiveActive.ORSetDeltaGroup.Entry parsedMessage = - null; + akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSetDeltaGroup.Entry + parsedMessage = null; try { parsedMessage = PARSER.parsePartialFrom(input, extensionRegistry); } catch (akka.protobufv3.internal.InvalidProtocolBufferException e) { parsedMessage = - (akka.persistence.typed.serialization.ActiveActive.ORSetDeltaGroup.Entry) + (akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSetDeltaGroup.Entry) e.getUnfinishedMessage(); throw e.unwrapIOException(); } finally { @@ -4150,12 +4266,14 @@ public final class ActiveActive { * * @return The operation. */ - public akka.persistence.typed.serialization.ActiveActive.ORSetDeltaOp getOperation() { + public akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSetDeltaOp + getOperation() { @SuppressWarnings("deprecation") - akka.persistence.typed.serialization.ActiveActive.ORSetDeltaOp result = - akka.persistence.typed.serialization.ActiveActive.ORSetDeltaOp.valueOf(operation_); + akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSetDeltaOp result = + akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSetDeltaOp.valueOf( + operation_); return result == null - ? akka.persistence.typed.serialization.ActiveActive.ORSetDeltaOp.Add + ? akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSetDeltaOp.Add : result; } /** @@ -4165,7 +4283,7 @@ public final class ActiveActive { * @return This builder for chaining. */ public Builder setOperation( - akka.persistence.typed.serialization.ActiveActive.ORSetDeltaOp value) { + akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSetDeltaOp value) { if (value == null) { throw new NullPointerException(); } @@ -4186,11 +4304,11 @@ public final class ActiveActive { return this; } - private akka.persistence.typed.serialization.ActiveActive.ORSet underlying_; + private akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSet underlying_; private akka.protobufv3.internal.SingleFieldBuilderV3< - akka.persistence.typed.serialization.ActiveActive.ORSet, - akka.persistence.typed.serialization.ActiveActive.ORSet.Builder, - akka.persistence.typed.serialization.ActiveActive.ORSetOrBuilder> + akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSet, + akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSet.Builder, + akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSetOrBuilder> underlyingBuilder_; /** * required .ORSet underlying = 2; @@ -4205,10 +4323,11 @@ public final class ActiveActive { * * @return The underlying. */ - public akka.persistence.typed.serialization.ActiveActive.ORSet getUnderlying() { + public akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSet getUnderlying() { if (underlyingBuilder_ == null) { return underlying_ == null - ? akka.persistence.typed.serialization.ActiveActive.ORSet.getDefaultInstance() + ? akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSet + .getDefaultInstance() : underlying_; } else { return underlyingBuilder_.getMessage(); @@ -4216,7 +4335,7 @@ public final class ActiveActive { } /** required .ORSet underlying = 2; */ public Builder setUnderlying( - akka.persistence.typed.serialization.ActiveActive.ORSet value) { + akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSet value) { if (underlyingBuilder_ == null) { if (value == null) { throw new NullPointerException(); @@ -4231,7 +4350,8 @@ public final class ActiveActive { } /** required .ORSet underlying = 2; */ public Builder setUnderlying( - akka.persistence.typed.serialization.ActiveActive.ORSet.Builder builderForValue) { + akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSet.Builder + builderForValue) { if (underlyingBuilder_ == null) { underlying_ = builderForValue.build(); onChanged(); @@ -4243,15 +4363,16 @@ public final class ActiveActive { } /** required .ORSet underlying = 2; */ public Builder mergeUnderlying( - akka.persistence.typed.serialization.ActiveActive.ORSet value) { + akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSet value) { if (underlyingBuilder_ == null) { if (((bitField0_ & 0x00000002) != 0) && underlying_ != null && underlying_ - != akka.persistence.typed.serialization.ActiveActive.ORSet + != akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSet .getDefaultInstance()) { underlying_ = - akka.persistence.typed.serialization.ActiveActive.ORSet.newBuilder(underlying_) + akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSet.newBuilder( + underlying_) .mergeFrom(value) .buildPartial(); } else { @@ -4276,35 +4397,36 @@ public final class ActiveActive { return this; } /** required .ORSet underlying = 2; */ - public akka.persistence.typed.serialization.ActiveActive.ORSet.Builder + public akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSet.Builder getUnderlyingBuilder() { bitField0_ |= 0x00000002; onChanged(); return getUnderlyingFieldBuilder().getBuilder(); } /** required .ORSet underlying = 2; */ - public akka.persistence.typed.serialization.ActiveActive.ORSetOrBuilder + public akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSetOrBuilder getUnderlyingOrBuilder() { if (underlyingBuilder_ != null) { return underlyingBuilder_.getMessageOrBuilder(); } else { return underlying_ == null - ? akka.persistence.typed.serialization.ActiveActive.ORSet.getDefaultInstance() + ? akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSet + .getDefaultInstance() : underlying_; } } /** required .ORSet underlying = 2; */ private akka.protobufv3.internal.SingleFieldBuilderV3< - akka.persistence.typed.serialization.ActiveActive.ORSet, - akka.persistence.typed.serialization.ActiveActive.ORSet.Builder, - akka.persistence.typed.serialization.ActiveActive.ORSetOrBuilder> + akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSet, + akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSet.Builder, + akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSetOrBuilder> getUnderlyingFieldBuilder() { if (underlyingBuilder_ == null) { underlyingBuilder_ = new akka.protobufv3.internal.SingleFieldBuilderV3< - akka.persistence.typed.serialization.ActiveActive.ORSet, - akka.persistence.typed.serialization.ActiveActive.ORSet.Builder, - akka.persistence.typed.serialization.ActiveActive.ORSetOrBuilder>( + akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSet, + akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSet.Builder, + akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSetOrBuilder>( getUnderlying(), getParentForChildren(), isClean()); underlying_ = null; } @@ -4327,15 +4449,18 @@ public final class ActiveActive { } // @@protoc_insertion_point(class_scope:ORSetDeltaGroup.Entry) - private static final akka.persistence.typed.serialization.ActiveActive.ORSetDeltaGroup.Entry + private static final akka.persistence.typed.serialization.ReplicatedEventSourcing + .ORSetDeltaGroup.Entry DEFAULT_INSTANCE; static { DEFAULT_INSTANCE = - new akka.persistence.typed.serialization.ActiveActive.ORSetDeltaGroup.Entry(); + new akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSetDeltaGroup + .Entry(); } - public static akka.persistence.typed.serialization.ActiveActive.ORSetDeltaGroup.Entry + public static akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSetDeltaGroup + .Entry getDefaultInstance() { return DEFAULT_INSTANCE; } @@ -4362,24 +4487,27 @@ public final class ActiveActive { } @java.lang.Override - public akka.persistence.typed.serialization.ActiveActive.ORSetDeltaGroup.Entry + public akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSetDeltaGroup.Entry getDefaultInstanceForType() { return DEFAULT_INSTANCE; } } public static final int ENTRIES_FIELD_NUMBER = 1; - private java.util.List + private java.util.List< + akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSetDeltaGroup.Entry> entries_; /** repeated .ORSetDeltaGroup.Entry entries = 1; */ - public java.util.List + public java.util.List< + akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSetDeltaGroup.Entry> getEntriesList() { return entries_; } /** repeated .ORSetDeltaGroup.Entry entries = 1; */ public java.util.List< ? extends - akka.persistence.typed.serialization.ActiveActive.ORSetDeltaGroup.EntryOrBuilder> + akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSetDeltaGroup + .EntryOrBuilder> getEntriesOrBuilderList() { return entries_; } @@ -4388,12 +4516,13 @@ public final class ActiveActive { return entries_.size(); } /** repeated .ORSetDeltaGroup.Entry entries = 1; */ - public akka.persistence.typed.serialization.ActiveActive.ORSetDeltaGroup.Entry getEntries( - int index) { + public akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSetDeltaGroup.Entry + getEntries(int index) { return entries_.get(index); } /** repeated .ORSetDeltaGroup.Entry entries = 1; */ - public akka.persistence.typed.serialization.ActiveActive.ORSetDeltaGroup.EntryOrBuilder + public akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSetDeltaGroup + .EntryOrBuilder getEntriesOrBuilder(int index) { return entries_.get(index); } @@ -4444,11 +4573,13 @@ public final class ActiveActive { if (obj == this) { return true; } - if (!(obj instanceof akka.persistence.typed.serialization.ActiveActive.ORSetDeltaGroup)) { + if (!(obj + instanceof + akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSetDeltaGroup)) { return super.equals(obj); } - akka.persistence.typed.serialization.ActiveActive.ORSetDeltaGroup other = - (akka.persistence.typed.serialization.ActiveActive.ORSetDeltaGroup) obj; + akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSetDeltaGroup other = + (akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSetDeltaGroup) obj; if (!getEntriesList().equals(other.getEntriesList())) return false; if (!unknownFields.equals(other.unknownFields)) return false; @@ -4471,60 +4602,66 @@ public final class ActiveActive { return hash; } - public static akka.persistence.typed.serialization.ActiveActive.ORSetDeltaGroup parseFrom( - java.nio.ByteBuffer data) throws akka.protobufv3.internal.InvalidProtocolBufferException { + public static akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSetDeltaGroup + parseFrom(java.nio.ByteBuffer data) + throws akka.protobufv3.internal.InvalidProtocolBufferException { return PARSER.parseFrom(data); } - public static akka.persistence.typed.serialization.ActiveActive.ORSetDeltaGroup parseFrom( - java.nio.ByteBuffer data, akka.protobufv3.internal.ExtensionRegistryLite extensionRegistry) - throws akka.protobufv3.internal.InvalidProtocolBufferException { + public static akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSetDeltaGroup + parseFrom( + java.nio.ByteBuffer data, + akka.protobufv3.internal.ExtensionRegistryLite extensionRegistry) + throws akka.protobufv3.internal.InvalidProtocolBufferException { return PARSER.parseFrom(data, extensionRegistry); } - public static akka.persistence.typed.serialization.ActiveActive.ORSetDeltaGroup parseFrom( - akka.protobufv3.internal.ByteString data) - throws akka.protobufv3.internal.InvalidProtocolBufferException { + public static akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSetDeltaGroup + parseFrom(akka.protobufv3.internal.ByteString data) + throws akka.protobufv3.internal.InvalidProtocolBufferException { return PARSER.parseFrom(data); } - public static akka.persistence.typed.serialization.ActiveActive.ORSetDeltaGroup parseFrom( - akka.protobufv3.internal.ByteString data, - akka.protobufv3.internal.ExtensionRegistryLite extensionRegistry) - throws akka.protobufv3.internal.InvalidProtocolBufferException { + public static akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSetDeltaGroup + parseFrom( + akka.protobufv3.internal.ByteString data, + akka.protobufv3.internal.ExtensionRegistryLite extensionRegistry) + throws akka.protobufv3.internal.InvalidProtocolBufferException { return PARSER.parseFrom(data, extensionRegistry); } - public static akka.persistence.typed.serialization.ActiveActive.ORSetDeltaGroup parseFrom( - byte[] data) throws akka.protobufv3.internal.InvalidProtocolBufferException { + public static akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSetDeltaGroup + parseFrom(byte[] data) throws akka.protobufv3.internal.InvalidProtocolBufferException { return PARSER.parseFrom(data); } - public static akka.persistence.typed.serialization.ActiveActive.ORSetDeltaGroup parseFrom( - byte[] data, akka.protobufv3.internal.ExtensionRegistryLite extensionRegistry) - throws akka.protobufv3.internal.InvalidProtocolBufferException { + public static akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSetDeltaGroup + parseFrom(byte[] data, akka.protobufv3.internal.ExtensionRegistryLite extensionRegistry) + throws akka.protobufv3.internal.InvalidProtocolBufferException { return PARSER.parseFrom(data, extensionRegistry); } - public static akka.persistence.typed.serialization.ActiveActive.ORSetDeltaGroup parseFrom( - java.io.InputStream input) throws java.io.IOException { + public static akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSetDeltaGroup + parseFrom(java.io.InputStream input) throws java.io.IOException { return akka.protobufv3.internal.GeneratedMessageV3.parseWithIOException(PARSER, input); } - public static akka.persistence.typed.serialization.ActiveActive.ORSetDeltaGroup parseFrom( - java.io.InputStream input, akka.protobufv3.internal.ExtensionRegistryLite extensionRegistry) - throws java.io.IOException { + public static akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSetDeltaGroup + parseFrom( + java.io.InputStream input, + akka.protobufv3.internal.ExtensionRegistryLite extensionRegistry) + throws java.io.IOException { return akka.protobufv3.internal.GeneratedMessageV3.parseWithIOException( PARSER, input, extensionRegistry); } - public static akka.persistence.typed.serialization.ActiveActive.ORSetDeltaGroup + public static akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSetDeltaGroup parseDelimitedFrom(java.io.InputStream input) throws java.io.IOException { return akka.protobufv3.internal.GeneratedMessageV3.parseDelimitedWithIOException( PARSER, input); } - public static akka.persistence.typed.serialization.ActiveActive.ORSetDeltaGroup + public static akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSetDeltaGroup parseDelimitedFrom( java.io.InputStream input, akka.protobufv3.internal.ExtensionRegistryLite extensionRegistry) @@ -4533,15 +4670,16 @@ public final class ActiveActive { PARSER, input, extensionRegistry); } - public static akka.persistence.typed.serialization.ActiveActive.ORSetDeltaGroup parseFrom( - akka.protobufv3.internal.CodedInputStream input) throws java.io.IOException { + public static akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSetDeltaGroup + parseFrom(akka.protobufv3.internal.CodedInputStream input) throws java.io.IOException { return akka.protobufv3.internal.GeneratedMessageV3.parseWithIOException(PARSER, input); } - public static akka.persistence.typed.serialization.ActiveActive.ORSetDeltaGroup parseFrom( - akka.protobufv3.internal.CodedInputStream input, - akka.protobufv3.internal.ExtensionRegistryLite extensionRegistry) - throws java.io.IOException { + public static akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSetDeltaGroup + parseFrom( + akka.protobufv3.internal.CodedInputStream input, + akka.protobufv3.internal.ExtensionRegistryLite extensionRegistry) + throws java.io.IOException { return akka.protobufv3.internal.GeneratedMessageV3.parseWithIOException( PARSER, input, extensionRegistry); } @@ -4556,7 +4694,7 @@ public final class ActiveActive { } public static Builder newBuilder( - akka.persistence.typed.serialization.ActiveActive.ORSetDeltaGroup prototype) { + akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSetDeltaGroup prototype) { return DEFAULT_INSTANCE.toBuilder().mergeFrom(prototype); } @@ -4576,24 +4714,25 @@ public final class ActiveActive { extends akka.protobufv3.internal.GeneratedMessageV3.Builder implements // @@protoc_insertion_point(builder_implements:ORSetDeltaGroup) - akka.persistence.typed.serialization.ActiveActive.ORSetDeltaGroupOrBuilder { + akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSetDeltaGroupOrBuilder { public static final akka.protobufv3.internal.Descriptors.Descriptor getDescriptor() { - return akka.persistence.typed.serialization.ActiveActive + return akka.persistence.typed.serialization.ReplicatedEventSourcing .internal_static_ORSetDeltaGroup_descriptor; } @java.lang.Override protected akka.protobufv3.internal.GeneratedMessageV3.FieldAccessorTable internalGetFieldAccessorTable() { - return akka.persistence.typed.serialization.ActiveActive + return akka.persistence.typed.serialization.ReplicatedEventSourcing .internal_static_ORSetDeltaGroup_fieldAccessorTable .ensureFieldAccessorsInitialized( - akka.persistence.typed.serialization.ActiveActive.ORSetDeltaGroup.class, - akka.persistence.typed.serialization.ActiveActive.ORSetDeltaGroup.Builder.class); + akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSetDeltaGroup.class, + akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSetDeltaGroup.Builder + .class); } // Construct using - // akka.persistence.typed.serialization.ActiveActive.ORSetDeltaGroup.newBuilder() + // akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSetDeltaGroup.newBuilder() private Builder() { maybeForceBuilderInitialization(); } @@ -4623,20 +4762,21 @@ public final class ActiveActive { @java.lang.Override public akka.protobufv3.internal.Descriptors.Descriptor getDescriptorForType() { - return akka.persistence.typed.serialization.ActiveActive + return akka.persistence.typed.serialization.ReplicatedEventSourcing .internal_static_ORSetDeltaGroup_descriptor; } @java.lang.Override - public akka.persistence.typed.serialization.ActiveActive.ORSetDeltaGroup + public akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSetDeltaGroup getDefaultInstanceForType() { - return akka.persistence.typed.serialization.ActiveActive.ORSetDeltaGroup + return akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSetDeltaGroup .getDefaultInstance(); } @java.lang.Override - public akka.persistence.typed.serialization.ActiveActive.ORSetDeltaGroup build() { - akka.persistence.typed.serialization.ActiveActive.ORSetDeltaGroup result = buildPartial(); + public akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSetDeltaGroup build() { + akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSetDeltaGroup result = + buildPartial(); if (!result.isInitialized()) { throw newUninitializedMessageException(result); } @@ -4644,9 +4784,10 @@ public final class ActiveActive { } @java.lang.Override - public akka.persistence.typed.serialization.ActiveActive.ORSetDeltaGroup buildPartial() { - akka.persistence.typed.serialization.ActiveActive.ORSetDeltaGroup result = - new akka.persistence.typed.serialization.ActiveActive.ORSetDeltaGroup(this); + public akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSetDeltaGroup + buildPartial() { + akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSetDeltaGroup result = + new akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSetDeltaGroup(this); int from_bitField0_ = bitField0_; if (entriesBuilder_ == null) { if (((bitField0_ & 0x00000001) != 0)) { @@ -4698,9 +4839,11 @@ public final class ActiveActive { @java.lang.Override public Builder mergeFrom(akka.protobufv3.internal.Message other) { - if (other instanceof akka.persistence.typed.serialization.ActiveActive.ORSetDeltaGroup) { + if (other + instanceof + akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSetDeltaGroup) { return mergeFrom( - (akka.persistence.typed.serialization.ActiveActive.ORSetDeltaGroup) other); + (akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSetDeltaGroup) other); } else { super.mergeFrom(other); return this; @@ -4708,9 +4851,9 @@ public final class ActiveActive { } public Builder mergeFrom( - akka.persistence.typed.serialization.ActiveActive.ORSetDeltaGroup other) { + akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSetDeltaGroup other) { if (other - == akka.persistence.typed.serialization.ActiveActive.ORSetDeltaGroup + == akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSetDeltaGroup .getDefaultInstance()) return this; if (entriesBuilder_ == null) { if (!other.entries_.isEmpty()) { @@ -4759,12 +4902,13 @@ public final class ActiveActive { akka.protobufv3.internal.CodedInputStream input, akka.protobufv3.internal.ExtensionRegistryLite extensionRegistry) throws java.io.IOException { - akka.persistence.typed.serialization.ActiveActive.ORSetDeltaGroup parsedMessage = null; + akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSetDeltaGroup parsedMessage = + null; try { parsedMessage = PARSER.parsePartialFrom(input, extensionRegistry); } catch (akka.protobufv3.internal.InvalidProtocolBufferException e) { parsedMessage = - (akka.persistence.typed.serialization.ActiveActive.ORSetDeltaGroup) + (akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSetDeltaGroup) e.getUnfinishedMessage(); throw e.unwrapIOException(); } finally { @@ -4778,27 +4922,30 @@ public final class ActiveActive { private int bitField0_; private java.util.List< - akka.persistence.typed.serialization.ActiveActive.ORSetDeltaGroup.Entry> + akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSetDeltaGroup.Entry> entries_ = java.util.Collections.emptyList(); private void ensureEntriesIsMutable() { if (!((bitField0_ & 0x00000001) != 0)) { entries_ = new java.util.ArrayList< - akka.persistence.typed.serialization.ActiveActive.ORSetDeltaGroup.Entry>( - entries_); + akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSetDeltaGroup + .Entry>(entries_); bitField0_ |= 0x00000001; } } private akka.protobufv3.internal.RepeatedFieldBuilderV3< - akka.persistence.typed.serialization.ActiveActive.ORSetDeltaGroup.Entry, - akka.persistence.typed.serialization.ActiveActive.ORSetDeltaGroup.Entry.Builder, - akka.persistence.typed.serialization.ActiveActive.ORSetDeltaGroup.EntryOrBuilder> + akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSetDeltaGroup.Entry, + akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSetDeltaGroup.Entry + .Builder, + akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSetDeltaGroup + .EntryOrBuilder> entriesBuilder_; /** repeated .ORSetDeltaGroup.Entry entries = 1; */ - public java.util.List + public java.util.List< + akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSetDeltaGroup.Entry> getEntriesList() { if (entriesBuilder_ == null) { return java.util.Collections.unmodifiableList(entries_); @@ -4815,8 +4962,8 @@ public final class ActiveActive { } } /** repeated .ORSetDeltaGroup.Entry entries = 1; */ - public akka.persistence.typed.serialization.ActiveActive.ORSetDeltaGroup.Entry getEntries( - int index) { + public akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSetDeltaGroup.Entry + getEntries(int index) { if (entriesBuilder_ == null) { return entries_.get(index); } else { @@ -4826,7 +4973,8 @@ public final class ActiveActive { /** repeated .ORSetDeltaGroup.Entry entries = 1; */ public Builder setEntries( int index, - akka.persistence.typed.serialization.ActiveActive.ORSetDeltaGroup.Entry value) { + akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSetDeltaGroup.Entry + value) { if (entriesBuilder_ == null) { if (value == null) { throw new NullPointerException(); @@ -4842,7 +4990,7 @@ public final class ActiveActive { /** repeated .ORSetDeltaGroup.Entry entries = 1; */ public Builder setEntries( int index, - akka.persistence.typed.serialization.ActiveActive.ORSetDeltaGroup.Entry.Builder + akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSetDeltaGroup.Entry.Builder builderForValue) { if (entriesBuilder_ == null) { ensureEntriesIsMutable(); @@ -4855,7 +5003,8 @@ public final class ActiveActive { } /** repeated .ORSetDeltaGroup.Entry entries = 1; */ public Builder addEntries( - akka.persistence.typed.serialization.ActiveActive.ORSetDeltaGroup.Entry value) { + akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSetDeltaGroup.Entry + value) { if (entriesBuilder_ == null) { if (value == null) { throw new NullPointerException(); @@ -4871,7 +5020,8 @@ public final class ActiveActive { /** repeated .ORSetDeltaGroup.Entry entries = 1; */ public Builder addEntries( int index, - akka.persistence.typed.serialization.ActiveActive.ORSetDeltaGroup.Entry value) { + akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSetDeltaGroup.Entry + value) { if (entriesBuilder_ == null) { if (value == null) { throw new NullPointerException(); @@ -4886,7 +5036,7 @@ public final class ActiveActive { } /** repeated .ORSetDeltaGroup.Entry entries = 1; */ public Builder addEntries( - akka.persistence.typed.serialization.ActiveActive.ORSetDeltaGroup.Entry.Builder + akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSetDeltaGroup.Entry.Builder builderForValue) { if (entriesBuilder_ == null) { ensureEntriesIsMutable(); @@ -4900,7 +5050,7 @@ public final class ActiveActive { /** repeated .ORSetDeltaGroup.Entry entries = 1; */ public Builder addEntries( int index, - akka.persistence.typed.serialization.ActiveActive.ORSetDeltaGroup.Entry.Builder + akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSetDeltaGroup.Entry.Builder builderForValue) { if (entriesBuilder_ == null) { ensureEntriesIsMutable(); @@ -4914,7 +5064,9 @@ public final class ActiveActive { /** repeated .ORSetDeltaGroup.Entry entries = 1; */ public Builder addAllEntries( java.lang.Iterable< - ? extends akka.persistence.typed.serialization.ActiveActive.ORSetDeltaGroup.Entry> + ? extends + akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSetDeltaGroup + .Entry> values) { if (entriesBuilder_ == null) { ensureEntriesIsMutable(); @@ -4948,12 +5100,14 @@ public final class ActiveActive { return this; } /** repeated .ORSetDeltaGroup.Entry entries = 1; */ - public akka.persistence.typed.serialization.ActiveActive.ORSetDeltaGroup.Entry.Builder + public akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSetDeltaGroup.Entry + .Builder getEntriesBuilder(int index) { return getEntriesFieldBuilder().getBuilder(index); } /** repeated .ORSetDeltaGroup.Entry entries = 1; */ - public akka.persistence.typed.serialization.ActiveActive.ORSetDeltaGroup.EntryOrBuilder + public akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSetDeltaGroup + .EntryOrBuilder getEntriesOrBuilder(int index) { if (entriesBuilder_ == null) { return entries_.get(index); @@ -4964,7 +5118,8 @@ public final class ActiveActive { /** repeated .ORSetDeltaGroup.Entry entries = 1; */ public java.util.List< ? extends - akka.persistence.typed.serialization.ActiveActive.ORSetDeltaGroup.EntryOrBuilder> + akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSetDeltaGroup + .EntryOrBuilder> getEntriesOrBuilderList() { if (entriesBuilder_ != null) { return entriesBuilder_.getMessageOrBuilderList(); @@ -4973,40 +5128,48 @@ public final class ActiveActive { } } /** repeated .ORSetDeltaGroup.Entry entries = 1; */ - public akka.persistence.typed.serialization.ActiveActive.ORSetDeltaGroup.Entry.Builder + public akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSetDeltaGroup.Entry + .Builder addEntriesBuilder() { return getEntriesFieldBuilder() .addBuilder( - akka.persistence.typed.serialization.ActiveActive.ORSetDeltaGroup.Entry + akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSetDeltaGroup.Entry .getDefaultInstance()); } /** repeated .ORSetDeltaGroup.Entry entries = 1; */ - public akka.persistence.typed.serialization.ActiveActive.ORSetDeltaGroup.Entry.Builder + public akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSetDeltaGroup.Entry + .Builder addEntriesBuilder(int index) { return getEntriesFieldBuilder() .addBuilder( index, - akka.persistence.typed.serialization.ActiveActive.ORSetDeltaGroup.Entry + akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSetDeltaGroup.Entry .getDefaultInstance()); } /** repeated .ORSetDeltaGroup.Entry entries = 1; */ public java.util.List< - akka.persistence.typed.serialization.ActiveActive.ORSetDeltaGroup.Entry.Builder> + akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSetDeltaGroup.Entry + .Builder> getEntriesBuilderList() { return getEntriesFieldBuilder().getBuilderList(); } private akka.protobufv3.internal.RepeatedFieldBuilderV3< - akka.persistence.typed.serialization.ActiveActive.ORSetDeltaGroup.Entry, - akka.persistence.typed.serialization.ActiveActive.ORSetDeltaGroup.Entry.Builder, - akka.persistence.typed.serialization.ActiveActive.ORSetDeltaGroup.EntryOrBuilder> + akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSetDeltaGroup.Entry, + akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSetDeltaGroup.Entry + .Builder, + akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSetDeltaGroup + .EntryOrBuilder> getEntriesFieldBuilder() { if (entriesBuilder_ == null) { entriesBuilder_ = new akka.protobufv3.internal.RepeatedFieldBuilderV3< - akka.persistence.typed.serialization.ActiveActive.ORSetDeltaGroup.Entry, - akka.persistence.typed.serialization.ActiveActive.ORSetDeltaGroup.Entry.Builder, - akka.persistence.typed.serialization.ActiveActive.ORSetDeltaGroup.EntryOrBuilder>( + akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSetDeltaGroup + .Entry, + akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSetDeltaGroup.Entry + .Builder, + akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSetDeltaGroup + .EntryOrBuilder>( entries_, ((bitField0_ & 0x00000001) != 0), getParentForChildren(), isClean()); entries_ = null; } @@ -5029,14 +5192,16 @@ public final class ActiveActive { } // @@protoc_insertion_point(class_scope:ORSetDeltaGroup) - private static final akka.persistence.typed.serialization.ActiveActive.ORSetDeltaGroup + private static final akka.persistence.typed.serialization.ReplicatedEventSourcing + .ORSetDeltaGroup DEFAULT_INSTANCE; static { - DEFAULT_INSTANCE = new akka.persistence.typed.serialization.ActiveActive.ORSetDeltaGroup(); + DEFAULT_INSTANCE = + new akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSetDeltaGroup(); } - public static akka.persistence.typed.serialization.ActiveActive.ORSetDeltaGroup + public static akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSetDeltaGroup getDefaultInstance() { return DEFAULT_INSTANCE; } @@ -5063,7 +5228,7 @@ public final class ActiveActive { } @java.lang.Override - public akka.persistence.typed.serialization.ActiveActive.ORSetDeltaGroup + public akka.persistence.typed.serialization.ReplicatedEventSourcing.ORSetDeltaGroup getDefaultInstanceForType() { return DEFAULT_INSTANCE; } @@ -5075,19 +5240,21 @@ public final class ActiveActive { akka.protobufv3.internal.MessageOrBuilder { /** repeated .VersionVector.Entry entries = 1; */ - java.util.List + java.util.List getEntriesList(); /** repeated .VersionVector.Entry entries = 1; */ - akka.persistence.typed.serialization.ActiveActive.VersionVector.Entry getEntries(int index); + akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector.Entry getEntries( + int index); /** repeated .VersionVector.Entry entries = 1; */ int getEntriesCount(); /** repeated .VersionVector.Entry entries = 1; */ java.util.List< ? extends - akka.persistence.typed.serialization.ActiveActive.VersionVector.EntryOrBuilder> + akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector + .EntryOrBuilder> getEntriesOrBuilderList(); /** repeated .VersionVector.Entry entries = 1; */ - akka.persistence.typed.serialization.ActiveActive.VersionVector.EntryOrBuilder + akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector.EntryOrBuilder getEntriesOrBuilder(int index); } /** Protobuf type {@code VersionVector} */ @@ -5141,13 +5308,14 @@ public final class ActiveActive { if (!((mutable_bitField0_ & 0x00000001) != 0)) { entries_ = new java.util.ArrayList< - akka.persistence.typed.serialization.ActiveActive.VersionVector.Entry>(); + akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector + .Entry>(); mutable_bitField0_ |= 0x00000001; } entries_.add( input.readMessage( - akka.persistence.typed.serialization.ActiveActive.VersionVector.Entry - .PARSER, + akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector + .Entry.PARSER, extensionRegistry)); break; } @@ -5175,18 +5343,19 @@ public final class ActiveActive { } public static final akka.protobufv3.internal.Descriptors.Descriptor getDescriptor() { - return akka.persistence.typed.serialization.ActiveActive + return akka.persistence.typed.serialization.ReplicatedEventSourcing .internal_static_VersionVector_descriptor; } @java.lang.Override protected akka.protobufv3.internal.GeneratedMessageV3.FieldAccessorTable internalGetFieldAccessorTable() { - return akka.persistence.typed.serialization.ActiveActive + return akka.persistence.typed.serialization.ReplicatedEventSourcing .internal_static_VersionVector_fieldAccessorTable .ensureFieldAccessorsInitialized( - akka.persistence.typed.serialization.ActiveActive.VersionVector.class, - akka.persistence.typed.serialization.ActiveActive.VersionVector.Builder.class); + akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector.class, + akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector.Builder + .class); } public interface EntryOrBuilder @@ -5306,19 +5475,20 @@ public final class ActiveActive { } public static final akka.protobufv3.internal.Descriptors.Descriptor getDescriptor() { - return akka.persistence.typed.serialization.ActiveActive + return akka.persistence.typed.serialization.ReplicatedEventSourcing .internal_static_VersionVector_Entry_descriptor; } @java.lang.Override protected akka.protobufv3.internal.GeneratedMessageV3.FieldAccessorTable internalGetFieldAccessorTable() { - return akka.persistence.typed.serialization.ActiveActive + return akka.persistence.typed.serialization.ReplicatedEventSourcing .internal_static_VersionVector_Entry_fieldAccessorTable .ensureFieldAccessorsInitialized( - akka.persistence.typed.serialization.ActiveActive.VersionVector.Entry.class, - akka.persistence.typed.serialization.ActiveActive.VersionVector.Entry.Builder - .class); + akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector.Entry + .class, + akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector.Entry + .Builder.class); } private int bitField0_; @@ -5441,11 +5611,12 @@ public final class ActiveActive { return true; } if (!(obj - instanceof akka.persistence.typed.serialization.ActiveActive.VersionVector.Entry)) { + instanceof + akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector.Entry)) { return super.equals(obj); } - akka.persistence.typed.serialization.ActiveActive.VersionVector.Entry other = - (akka.persistence.typed.serialization.ActiveActive.VersionVector.Entry) obj; + akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector.Entry other = + (akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector.Entry) obj; if (hasKey() != other.hasKey()) return false; if (hasKey()) { @@ -5479,62 +5650,66 @@ public final class ActiveActive { return hash; } - public static akka.persistence.typed.serialization.ActiveActive.VersionVector.Entry parseFrom( - java.nio.ByteBuffer data) throws akka.protobufv3.internal.InvalidProtocolBufferException { + public static akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector.Entry + parseFrom(java.nio.ByteBuffer data) + throws akka.protobufv3.internal.InvalidProtocolBufferException { return PARSER.parseFrom(data); } - public static akka.persistence.typed.serialization.ActiveActive.VersionVector.Entry parseFrom( - java.nio.ByteBuffer data, - akka.protobufv3.internal.ExtensionRegistryLite extensionRegistry) - throws akka.protobufv3.internal.InvalidProtocolBufferException { + public static akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector.Entry + parseFrom( + java.nio.ByteBuffer data, + akka.protobufv3.internal.ExtensionRegistryLite extensionRegistry) + throws akka.protobufv3.internal.InvalidProtocolBufferException { return PARSER.parseFrom(data, extensionRegistry); } - public static akka.persistence.typed.serialization.ActiveActive.VersionVector.Entry parseFrom( - akka.protobufv3.internal.ByteString data) - throws akka.protobufv3.internal.InvalidProtocolBufferException { + public static akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector.Entry + parseFrom(akka.protobufv3.internal.ByteString data) + throws akka.protobufv3.internal.InvalidProtocolBufferException { return PARSER.parseFrom(data); } - public static akka.persistence.typed.serialization.ActiveActive.VersionVector.Entry parseFrom( - akka.protobufv3.internal.ByteString data, - akka.protobufv3.internal.ExtensionRegistryLite extensionRegistry) - throws akka.protobufv3.internal.InvalidProtocolBufferException { + public static akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector.Entry + parseFrom( + akka.protobufv3.internal.ByteString data, + akka.protobufv3.internal.ExtensionRegistryLite extensionRegistry) + throws akka.protobufv3.internal.InvalidProtocolBufferException { return PARSER.parseFrom(data, extensionRegistry); } - public static akka.persistence.typed.serialization.ActiveActive.VersionVector.Entry parseFrom( - byte[] data) throws akka.protobufv3.internal.InvalidProtocolBufferException { + public static akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector.Entry + parseFrom(byte[] data) throws akka.protobufv3.internal.InvalidProtocolBufferException { return PARSER.parseFrom(data); } - public static akka.persistence.typed.serialization.ActiveActive.VersionVector.Entry parseFrom( - byte[] data, akka.protobufv3.internal.ExtensionRegistryLite extensionRegistry) - throws akka.protobufv3.internal.InvalidProtocolBufferException { + public static akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector.Entry + parseFrom(byte[] data, akka.protobufv3.internal.ExtensionRegistryLite extensionRegistry) + throws akka.protobufv3.internal.InvalidProtocolBufferException { return PARSER.parseFrom(data, extensionRegistry); } - public static akka.persistence.typed.serialization.ActiveActive.VersionVector.Entry parseFrom( - java.io.InputStream input) throws java.io.IOException { + public static akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector.Entry + parseFrom(java.io.InputStream input) throws java.io.IOException { return akka.protobufv3.internal.GeneratedMessageV3.parseWithIOException(PARSER, input); } - public static akka.persistence.typed.serialization.ActiveActive.VersionVector.Entry parseFrom( - java.io.InputStream input, - akka.protobufv3.internal.ExtensionRegistryLite extensionRegistry) - throws java.io.IOException { + public static akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector.Entry + parseFrom( + java.io.InputStream input, + akka.protobufv3.internal.ExtensionRegistryLite extensionRegistry) + throws java.io.IOException { return akka.protobufv3.internal.GeneratedMessageV3.parseWithIOException( PARSER, input, extensionRegistry); } - public static akka.persistence.typed.serialization.ActiveActive.VersionVector.Entry + public static akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector.Entry parseDelimitedFrom(java.io.InputStream input) throws java.io.IOException { return akka.protobufv3.internal.GeneratedMessageV3.parseDelimitedWithIOException( PARSER, input); } - public static akka.persistence.typed.serialization.ActiveActive.VersionVector.Entry + public static akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector.Entry parseDelimitedFrom( java.io.InputStream input, akka.protobufv3.internal.ExtensionRegistryLite extensionRegistry) @@ -5543,15 +5718,16 @@ public final class ActiveActive { PARSER, input, extensionRegistry); } - public static akka.persistence.typed.serialization.ActiveActive.VersionVector.Entry parseFrom( - akka.protobufv3.internal.CodedInputStream input) throws java.io.IOException { + public static akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector.Entry + parseFrom(akka.protobufv3.internal.CodedInputStream input) throws java.io.IOException { return akka.protobufv3.internal.GeneratedMessageV3.parseWithIOException(PARSER, input); } - public static akka.persistence.typed.serialization.ActiveActive.VersionVector.Entry parseFrom( - akka.protobufv3.internal.CodedInputStream input, - akka.protobufv3.internal.ExtensionRegistryLite extensionRegistry) - throws java.io.IOException { + public static akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector.Entry + parseFrom( + akka.protobufv3.internal.CodedInputStream input, + akka.protobufv3.internal.ExtensionRegistryLite extensionRegistry) + throws java.io.IOException { return akka.protobufv3.internal.GeneratedMessageV3.parseWithIOException( PARSER, input, extensionRegistry); } @@ -5566,7 +5742,8 @@ public final class ActiveActive { } public static Builder newBuilder( - akka.persistence.typed.serialization.ActiveActive.VersionVector.Entry prototype) { + akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector.Entry + prototype) { return DEFAULT_INSTANCE.toBuilder().mergeFrom(prototype); } @@ -5586,25 +5763,27 @@ public final class ActiveActive { extends akka.protobufv3.internal.GeneratedMessageV3.Builder implements // @@protoc_insertion_point(builder_implements:VersionVector.Entry) - akka.persistence.typed.serialization.ActiveActive.VersionVector.EntryOrBuilder { + akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector + .EntryOrBuilder { public static final akka.protobufv3.internal.Descriptors.Descriptor getDescriptor() { - return akka.persistence.typed.serialization.ActiveActive + return akka.persistence.typed.serialization.ReplicatedEventSourcing .internal_static_VersionVector_Entry_descriptor; } @java.lang.Override protected akka.protobufv3.internal.GeneratedMessageV3.FieldAccessorTable internalGetFieldAccessorTable() { - return akka.persistence.typed.serialization.ActiveActive + return akka.persistence.typed.serialization.ReplicatedEventSourcing .internal_static_VersionVector_Entry_fieldAccessorTable .ensureFieldAccessorsInitialized( - akka.persistence.typed.serialization.ActiveActive.VersionVector.Entry.class, - akka.persistence.typed.serialization.ActiveActive.VersionVector.Entry.Builder - .class); + akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector.Entry + .class, + akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector.Entry + .Builder.class); } // Construct using - // akka.persistence.typed.serialization.ActiveActive.VersionVector.Entry.newBuilder() + // akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector.Entry.newBuilder() private Builder() { maybeForceBuilderInitialization(); } @@ -5630,20 +5809,21 @@ public final class ActiveActive { @java.lang.Override public akka.protobufv3.internal.Descriptors.Descriptor getDescriptorForType() { - return akka.persistence.typed.serialization.ActiveActive + return akka.persistence.typed.serialization.ReplicatedEventSourcing .internal_static_VersionVector_Entry_descriptor; } @java.lang.Override - public akka.persistence.typed.serialization.ActiveActive.VersionVector.Entry + public akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector.Entry getDefaultInstanceForType() { - return akka.persistence.typed.serialization.ActiveActive.VersionVector.Entry + return akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector.Entry .getDefaultInstance(); } @java.lang.Override - public akka.persistence.typed.serialization.ActiveActive.VersionVector.Entry build() { - akka.persistence.typed.serialization.ActiveActive.VersionVector.Entry result = + public akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector.Entry + build() { + akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector.Entry result = buildPartial(); if (!result.isInitialized()) { throw newUninitializedMessageException(result); @@ -5652,10 +5832,11 @@ public final class ActiveActive { } @java.lang.Override - public akka.persistence.typed.serialization.ActiveActive.VersionVector.Entry + public akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector.Entry buildPartial() { - akka.persistence.typed.serialization.ActiveActive.VersionVector.Entry result = - new akka.persistence.typed.serialization.ActiveActive.VersionVector.Entry(this); + akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector.Entry result = + new akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector.Entry( + this); int from_bitField0_ = bitField0_; int to_bitField0_ = 0; if (((from_bitField0_ & 0x00000001) != 0)) { @@ -5709,9 +5890,11 @@ public final class ActiveActive { @java.lang.Override public Builder mergeFrom(akka.protobufv3.internal.Message other) { if (other - instanceof akka.persistence.typed.serialization.ActiveActive.VersionVector.Entry) { + instanceof + akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector.Entry) { return mergeFrom( - (akka.persistence.typed.serialization.ActiveActive.VersionVector.Entry) other); + (akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector.Entry) + other); } else { super.mergeFrom(other); return this; @@ -5719,9 +5902,10 @@ public final class ActiveActive { } public Builder mergeFrom( - akka.persistence.typed.serialization.ActiveActive.VersionVector.Entry other) { + akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector.Entry + other) { if (other - == akka.persistence.typed.serialization.ActiveActive.VersionVector.Entry + == akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector.Entry .getDefaultInstance()) return this; if (other.hasKey()) { bitField0_ |= 0x00000001; @@ -5752,13 +5936,13 @@ public final class ActiveActive { akka.protobufv3.internal.CodedInputStream input, akka.protobufv3.internal.ExtensionRegistryLite extensionRegistry) throws java.io.IOException { - akka.persistence.typed.serialization.ActiveActive.VersionVector.Entry parsedMessage = - null; + akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector.Entry + parsedMessage = null; try { parsedMessage = PARSER.parsePartialFrom(input, extensionRegistry); } catch (akka.protobufv3.internal.InvalidProtocolBufferException e) { parsedMessage = - (akka.persistence.typed.serialization.ActiveActive.VersionVector.Entry) + (akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector.Entry) e.getUnfinishedMessage(); throw e.unwrapIOException(); } finally { @@ -5913,15 +6097,16 @@ public final class ActiveActive { } // @@protoc_insertion_point(class_scope:VersionVector.Entry) - private static final akka.persistence.typed.serialization.ActiveActive.VersionVector.Entry + private static final akka.persistence.typed.serialization.ReplicatedEventSourcing + .VersionVector.Entry DEFAULT_INSTANCE; static { DEFAULT_INSTANCE = - new akka.persistence.typed.serialization.ActiveActive.VersionVector.Entry(); + new akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector.Entry(); } - public static akka.persistence.typed.serialization.ActiveActive.VersionVector.Entry + public static akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector.Entry getDefaultInstance() { return DEFAULT_INSTANCE; } @@ -5948,24 +6133,27 @@ public final class ActiveActive { } @java.lang.Override - public akka.persistence.typed.serialization.ActiveActive.VersionVector.Entry + public akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector.Entry getDefaultInstanceForType() { return DEFAULT_INSTANCE; } } public static final int ENTRIES_FIELD_NUMBER = 1; - private java.util.List + private java.util.List< + akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector.Entry> entries_; /** repeated .VersionVector.Entry entries = 1; */ - public java.util.List + public java.util.List< + akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector.Entry> getEntriesList() { return entries_; } /** repeated .VersionVector.Entry entries = 1; */ public java.util.List< ? extends - akka.persistence.typed.serialization.ActiveActive.VersionVector.EntryOrBuilder> + akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector + .EntryOrBuilder> getEntriesOrBuilderList() { return entries_; } @@ -5974,12 +6162,12 @@ public final class ActiveActive { return entries_.size(); } /** repeated .VersionVector.Entry entries = 1; */ - public akka.persistence.typed.serialization.ActiveActive.VersionVector.Entry getEntries( - int index) { + public akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector.Entry + getEntries(int index) { return entries_.get(index); } /** repeated .VersionVector.Entry entries = 1; */ - public akka.persistence.typed.serialization.ActiveActive.VersionVector.EntryOrBuilder + public akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector.EntryOrBuilder getEntriesOrBuilder(int index) { return entries_.get(index); } @@ -6030,11 +6218,12 @@ public final class ActiveActive { if (obj == this) { return true; } - if (!(obj instanceof akka.persistence.typed.serialization.ActiveActive.VersionVector)) { + if (!(obj + instanceof akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector)) { return super.equals(obj); } - akka.persistence.typed.serialization.ActiveActive.VersionVector other = - (akka.persistence.typed.serialization.ActiveActive.VersionVector) obj; + akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector other = + (akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector) obj; if (!getEntriesList().equals(other.getEntriesList())) return false; if (!unknownFields.equals(other.unknownFields)) return false; @@ -6057,60 +6246,66 @@ public final class ActiveActive { return hash; } - public static akka.persistence.typed.serialization.ActiveActive.VersionVector parseFrom( - java.nio.ByteBuffer data) throws akka.protobufv3.internal.InvalidProtocolBufferException { + public static akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector + parseFrom(java.nio.ByteBuffer data) + throws akka.protobufv3.internal.InvalidProtocolBufferException { return PARSER.parseFrom(data); } - public static akka.persistence.typed.serialization.ActiveActive.VersionVector parseFrom( - java.nio.ByteBuffer data, akka.protobufv3.internal.ExtensionRegistryLite extensionRegistry) - throws akka.protobufv3.internal.InvalidProtocolBufferException { + public static akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector + parseFrom( + java.nio.ByteBuffer data, + akka.protobufv3.internal.ExtensionRegistryLite extensionRegistry) + throws akka.protobufv3.internal.InvalidProtocolBufferException { return PARSER.parseFrom(data, extensionRegistry); } - public static akka.persistence.typed.serialization.ActiveActive.VersionVector parseFrom( - akka.protobufv3.internal.ByteString data) - throws akka.protobufv3.internal.InvalidProtocolBufferException { + public static akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector + parseFrom(akka.protobufv3.internal.ByteString data) + throws akka.protobufv3.internal.InvalidProtocolBufferException { return PARSER.parseFrom(data); } - public static akka.persistence.typed.serialization.ActiveActive.VersionVector parseFrom( - akka.protobufv3.internal.ByteString data, - akka.protobufv3.internal.ExtensionRegistryLite extensionRegistry) - throws akka.protobufv3.internal.InvalidProtocolBufferException { + public static akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector + parseFrom( + akka.protobufv3.internal.ByteString data, + akka.protobufv3.internal.ExtensionRegistryLite extensionRegistry) + throws akka.protobufv3.internal.InvalidProtocolBufferException { return PARSER.parseFrom(data, extensionRegistry); } - public static akka.persistence.typed.serialization.ActiveActive.VersionVector parseFrom( - byte[] data) throws akka.protobufv3.internal.InvalidProtocolBufferException { + public static akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector + parseFrom(byte[] data) throws akka.protobufv3.internal.InvalidProtocolBufferException { return PARSER.parseFrom(data); } - public static akka.persistence.typed.serialization.ActiveActive.VersionVector parseFrom( - byte[] data, akka.protobufv3.internal.ExtensionRegistryLite extensionRegistry) - throws akka.protobufv3.internal.InvalidProtocolBufferException { + public static akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector + parseFrom(byte[] data, akka.protobufv3.internal.ExtensionRegistryLite extensionRegistry) + throws akka.protobufv3.internal.InvalidProtocolBufferException { return PARSER.parseFrom(data, extensionRegistry); } - public static akka.persistence.typed.serialization.ActiveActive.VersionVector parseFrom( - java.io.InputStream input) throws java.io.IOException { + public static akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector + parseFrom(java.io.InputStream input) throws java.io.IOException { return akka.protobufv3.internal.GeneratedMessageV3.parseWithIOException(PARSER, input); } - public static akka.persistence.typed.serialization.ActiveActive.VersionVector parseFrom( - java.io.InputStream input, akka.protobufv3.internal.ExtensionRegistryLite extensionRegistry) - throws java.io.IOException { + public static akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector + parseFrom( + java.io.InputStream input, + akka.protobufv3.internal.ExtensionRegistryLite extensionRegistry) + throws java.io.IOException { return akka.protobufv3.internal.GeneratedMessageV3.parseWithIOException( PARSER, input, extensionRegistry); } - public static akka.persistence.typed.serialization.ActiveActive.VersionVector + public static akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector parseDelimitedFrom(java.io.InputStream input) throws java.io.IOException { return akka.protobufv3.internal.GeneratedMessageV3.parseDelimitedWithIOException( PARSER, input); } - public static akka.persistence.typed.serialization.ActiveActive.VersionVector + public static akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector parseDelimitedFrom( java.io.InputStream input, akka.protobufv3.internal.ExtensionRegistryLite extensionRegistry) @@ -6119,15 +6314,16 @@ public final class ActiveActive { PARSER, input, extensionRegistry); } - public static akka.persistence.typed.serialization.ActiveActive.VersionVector parseFrom( - akka.protobufv3.internal.CodedInputStream input) throws java.io.IOException { + public static akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector + parseFrom(akka.protobufv3.internal.CodedInputStream input) throws java.io.IOException { return akka.protobufv3.internal.GeneratedMessageV3.parseWithIOException(PARSER, input); } - public static akka.persistence.typed.serialization.ActiveActive.VersionVector parseFrom( - akka.protobufv3.internal.CodedInputStream input, - akka.protobufv3.internal.ExtensionRegistryLite extensionRegistry) - throws java.io.IOException { + public static akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector + parseFrom( + akka.protobufv3.internal.CodedInputStream input, + akka.protobufv3.internal.ExtensionRegistryLite extensionRegistry) + throws java.io.IOException { return akka.protobufv3.internal.GeneratedMessageV3.parseWithIOException( PARSER, input, extensionRegistry); } @@ -6142,7 +6338,7 @@ public final class ActiveActive { } public static Builder newBuilder( - akka.persistence.typed.serialization.ActiveActive.VersionVector prototype) { + akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector prototype) { return DEFAULT_INSTANCE.toBuilder().mergeFrom(prototype); } @@ -6162,24 +6358,25 @@ public final class ActiveActive { extends akka.protobufv3.internal.GeneratedMessageV3.Builder implements // @@protoc_insertion_point(builder_implements:VersionVector) - akka.persistence.typed.serialization.ActiveActive.VersionVectorOrBuilder { + akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVectorOrBuilder { public static final akka.protobufv3.internal.Descriptors.Descriptor getDescriptor() { - return akka.persistence.typed.serialization.ActiveActive + return akka.persistence.typed.serialization.ReplicatedEventSourcing .internal_static_VersionVector_descriptor; } @java.lang.Override protected akka.protobufv3.internal.GeneratedMessageV3.FieldAccessorTable internalGetFieldAccessorTable() { - return akka.persistence.typed.serialization.ActiveActive + return akka.persistence.typed.serialization.ReplicatedEventSourcing .internal_static_VersionVector_fieldAccessorTable .ensureFieldAccessorsInitialized( - akka.persistence.typed.serialization.ActiveActive.VersionVector.class, - akka.persistence.typed.serialization.ActiveActive.VersionVector.Builder.class); + akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector.class, + akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector.Builder + .class); } // Construct using - // akka.persistence.typed.serialization.ActiveActive.VersionVector.newBuilder() + // akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector.newBuilder() private Builder() { maybeForceBuilderInitialization(); } @@ -6209,19 +6406,21 @@ public final class ActiveActive { @java.lang.Override public akka.protobufv3.internal.Descriptors.Descriptor getDescriptorForType() { - return akka.persistence.typed.serialization.ActiveActive + return akka.persistence.typed.serialization.ReplicatedEventSourcing .internal_static_VersionVector_descriptor; } @java.lang.Override - public akka.persistence.typed.serialization.ActiveActive.VersionVector + public akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector getDefaultInstanceForType() { - return akka.persistence.typed.serialization.ActiveActive.VersionVector.getDefaultInstance(); + return akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector + .getDefaultInstance(); } @java.lang.Override - public akka.persistence.typed.serialization.ActiveActive.VersionVector build() { - akka.persistence.typed.serialization.ActiveActive.VersionVector result = buildPartial(); + public akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector build() { + akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector result = + buildPartial(); if (!result.isInitialized()) { throw newUninitializedMessageException(result); } @@ -6229,9 +6428,10 @@ public final class ActiveActive { } @java.lang.Override - public akka.persistence.typed.serialization.ActiveActive.VersionVector buildPartial() { - akka.persistence.typed.serialization.ActiveActive.VersionVector result = - new akka.persistence.typed.serialization.ActiveActive.VersionVector(this); + public akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector + buildPartial() { + akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector result = + new akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector(this); int from_bitField0_ = bitField0_; if (entriesBuilder_ == null) { if (((bitField0_ & 0x00000001) != 0)) { @@ -6283,8 +6483,10 @@ public final class ActiveActive { @java.lang.Override public Builder mergeFrom(akka.protobufv3.internal.Message other) { - if (other instanceof akka.persistence.typed.serialization.ActiveActive.VersionVector) { - return mergeFrom((akka.persistence.typed.serialization.ActiveActive.VersionVector) other); + if (other + instanceof akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector) { + return mergeFrom( + (akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector) other); } else { super.mergeFrom(other); return this; @@ -6292,10 +6494,10 @@ public final class ActiveActive { } public Builder mergeFrom( - akka.persistence.typed.serialization.ActiveActive.VersionVector other) { + akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector other) { if (other - == akka.persistence.typed.serialization.ActiveActive.VersionVector.getDefaultInstance()) - return this; + == akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector + .getDefaultInstance()) return this; if (entriesBuilder_ == null) { if (!other.entries_.isEmpty()) { if (entries_.isEmpty()) { @@ -6343,12 +6545,13 @@ public final class ActiveActive { akka.protobufv3.internal.CodedInputStream input, akka.protobufv3.internal.ExtensionRegistryLite extensionRegistry) throws java.io.IOException { - akka.persistence.typed.serialization.ActiveActive.VersionVector parsedMessage = null; + akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector parsedMessage = + null; try { parsedMessage = PARSER.parsePartialFrom(input, extensionRegistry); } catch (akka.protobufv3.internal.InvalidProtocolBufferException e) { parsedMessage = - (akka.persistence.typed.serialization.ActiveActive.VersionVector) + (akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector) e.getUnfinishedMessage(); throw e.unwrapIOException(); } finally { @@ -6361,26 +6564,31 @@ public final class ActiveActive { private int bitField0_; - private java.util.List + private java.util.List< + akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector.Entry> entries_ = java.util.Collections.emptyList(); private void ensureEntriesIsMutable() { if (!((bitField0_ & 0x00000001) != 0)) { entries_ = new java.util.ArrayList< - akka.persistence.typed.serialization.ActiveActive.VersionVector.Entry>(entries_); + akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector.Entry>( + entries_); bitField0_ |= 0x00000001; } } private akka.protobufv3.internal.RepeatedFieldBuilderV3< - akka.persistence.typed.serialization.ActiveActive.VersionVector.Entry, - akka.persistence.typed.serialization.ActiveActive.VersionVector.Entry.Builder, - akka.persistence.typed.serialization.ActiveActive.VersionVector.EntryOrBuilder> + akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector.Entry, + akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector.Entry + .Builder, + akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector + .EntryOrBuilder> entriesBuilder_; /** repeated .VersionVector.Entry entries = 1; */ - public java.util.List + public java.util.List< + akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector.Entry> getEntriesList() { if (entriesBuilder_ == null) { return java.util.Collections.unmodifiableList(entries_); @@ -6397,8 +6605,8 @@ public final class ActiveActive { } } /** repeated .VersionVector.Entry entries = 1; */ - public akka.persistence.typed.serialization.ActiveActive.VersionVector.Entry getEntries( - int index) { + public akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector.Entry + getEntries(int index) { if (entriesBuilder_ == null) { return entries_.get(index); } else { @@ -6407,7 +6615,8 @@ public final class ActiveActive { } /** repeated .VersionVector.Entry entries = 1; */ public Builder setEntries( - int index, akka.persistence.typed.serialization.ActiveActive.VersionVector.Entry value) { + int index, + akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector.Entry value) { if (entriesBuilder_ == null) { if (value == null) { throw new NullPointerException(); @@ -6423,7 +6632,7 @@ public final class ActiveActive { /** repeated .VersionVector.Entry entries = 1; */ public Builder setEntries( int index, - akka.persistence.typed.serialization.ActiveActive.VersionVector.Entry.Builder + akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector.Entry.Builder builderForValue) { if (entriesBuilder_ == null) { ensureEntriesIsMutable(); @@ -6436,7 +6645,7 @@ public final class ActiveActive { } /** repeated .VersionVector.Entry entries = 1; */ public Builder addEntries( - akka.persistence.typed.serialization.ActiveActive.VersionVector.Entry value) { + akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector.Entry value) { if (entriesBuilder_ == null) { if (value == null) { throw new NullPointerException(); @@ -6451,7 +6660,8 @@ public final class ActiveActive { } /** repeated .VersionVector.Entry entries = 1; */ public Builder addEntries( - int index, akka.persistence.typed.serialization.ActiveActive.VersionVector.Entry value) { + int index, + akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector.Entry value) { if (entriesBuilder_ == null) { if (value == null) { throw new NullPointerException(); @@ -6466,7 +6676,7 @@ public final class ActiveActive { } /** repeated .VersionVector.Entry entries = 1; */ public Builder addEntries( - akka.persistence.typed.serialization.ActiveActive.VersionVector.Entry.Builder + akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector.Entry.Builder builderForValue) { if (entriesBuilder_ == null) { ensureEntriesIsMutable(); @@ -6480,7 +6690,7 @@ public final class ActiveActive { /** repeated .VersionVector.Entry entries = 1; */ public Builder addEntries( int index, - akka.persistence.typed.serialization.ActiveActive.VersionVector.Entry.Builder + akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector.Entry.Builder builderForValue) { if (entriesBuilder_ == null) { ensureEntriesIsMutable(); @@ -6494,7 +6704,9 @@ public final class ActiveActive { /** repeated .VersionVector.Entry entries = 1; */ public Builder addAllEntries( java.lang.Iterable< - ? extends akka.persistence.typed.serialization.ActiveActive.VersionVector.Entry> + ? extends + akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector + .Entry> values) { if (entriesBuilder_ == null) { ensureEntriesIsMutable(); @@ -6528,12 +6740,14 @@ public final class ActiveActive { return this; } /** repeated .VersionVector.Entry entries = 1; */ - public akka.persistence.typed.serialization.ActiveActive.VersionVector.Entry.Builder + public akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector.Entry + .Builder getEntriesBuilder(int index) { return getEntriesFieldBuilder().getBuilder(index); } /** repeated .VersionVector.Entry entries = 1; */ - public akka.persistence.typed.serialization.ActiveActive.VersionVector.EntryOrBuilder + public akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector + .EntryOrBuilder getEntriesOrBuilder(int index) { if (entriesBuilder_ == null) { return entries_.get(index); @@ -6544,7 +6758,8 @@ public final class ActiveActive { /** repeated .VersionVector.Entry entries = 1; */ public java.util.List< ? extends - akka.persistence.typed.serialization.ActiveActive.VersionVector.EntryOrBuilder> + akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector + .EntryOrBuilder> getEntriesOrBuilderList() { if (entriesBuilder_ != null) { return entriesBuilder_.getMessageOrBuilderList(); @@ -6553,40 +6768,47 @@ public final class ActiveActive { } } /** repeated .VersionVector.Entry entries = 1; */ - public akka.persistence.typed.serialization.ActiveActive.VersionVector.Entry.Builder + public akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector.Entry + .Builder addEntriesBuilder() { return getEntriesFieldBuilder() .addBuilder( - akka.persistence.typed.serialization.ActiveActive.VersionVector.Entry + akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector.Entry .getDefaultInstance()); } /** repeated .VersionVector.Entry entries = 1; */ - public akka.persistence.typed.serialization.ActiveActive.VersionVector.Entry.Builder + public akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector.Entry + .Builder addEntriesBuilder(int index) { return getEntriesFieldBuilder() .addBuilder( index, - akka.persistence.typed.serialization.ActiveActive.VersionVector.Entry + akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector.Entry .getDefaultInstance()); } /** repeated .VersionVector.Entry entries = 1; */ public java.util.List< - akka.persistence.typed.serialization.ActiveActive.VersionVector.Entry.Builder> + akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector.Entry + .Builder> getEntriesBuilderList() { return getEntriesFieldBuilder().getBuilderList(); } private akka.protobufv3.internal.RepeatedFieldBuilderV3< - akka.persistence.typed.serialization.ActiveActive.VersionVector.Entry, - akka.persistence.typed.serialization.ActiveActive.VersionVector.Entry.Builder, - akka.persistence.typed.serialization.ActiveActive.VersionVector.EntryOrBuilder> + akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector.Entry, + akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector.Entry + .Builder, + akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector + .EntryOrBuilder> getEntriesFieldBuilder() { if (entriesBuilder_ == null) { entriesBuilder_ = new akka.protobufv3.internal.RepeatedFieldBuilderV3< - akka.persistence.typed.serialization.ActiveActive.VersionVector.Entry, - akka.persistence.typed.serialization.ActiveActive.VersionVector.Entry.Builder, - akka.persistence.typed.serialization.ActiveActive.VersionVector.EntryOrBuilder>( + akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector.Entry, + akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector.Entry + .Builder, + akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector + .EntryOrBuilder>( entries_, ((bitField0_ & 0x00000001) != 0), getParentForChildren(), isClean()); entries_ = null; } @@ -6609,14 +6831,15 @@ public final class ActiveActive { } // @@protoc_insertion_point(class_scope:VersionVector) - private static final akka.persistence.typed.serialization.ActiveActive.VersionVector + private static final akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector DEFAULT_INSTANCE; static { - DEFAULT_INSTANCE = new akka.persistence.typed.serialization.ActiveActive.VersionVector(); + DEFAULT_INSTANCE = + new akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector(); } - public static akka.persistence.typed.serialization.ActiveActive.VersionVector + public static akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector getDefaultInstance() { return DEFAULT_INSTANCE; } @@ -6643,7 +6866,7 @@ public final class ActiveActive { } @java.lang.Override - public akka.persistence.typed.serialization.ActiveActive.VersionVector + public akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector getDefaultInstanceForType() { return DEFAULT_INSTANCE; } @@ -6697,9 +6920,9 @@ public final class ActiveActive { * * @return The versionVector. */ - akka.persistence.typed.serialization.ActiveActive.VersionVector getVersionVector(); + akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector getVersionVector(); /** required .VersionVector versionVector = 3; */ - akka.persistence.typed.serialization.ActiveActive.VersionVectorOrBuilder + akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVectorOrBuilder getVersionVectorOrBuilder(); /** @@ -6778,14 +7001,15 @@ public final class ActiveActive { } case 26: { - akka.persistence.typed.serialization.ActiveActive.VersionVector.Builder subBuilder = - null; + akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector.Builder + subBuilder = null; if (((bitField0_ & 0x00000004) != 0)) { subBuilder = versionVector_.toBuilder(); } versionVector_ = input.readMessage( - akka.persistence.typed.serialization.ActiveActive.VersionVector.PARSER, + akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector + .PARSER, extensionRegistry); if (subBuilder != null) { subBuilder.mergeFrom(versionVector_); @@ -6821,19 +7045,20 @@ public final class ActiveActive { } public static final akka.protobufv3.internal.Descriptors.Descriptor getDescriptor() { - return akka.persistence.typed.serialization.ActiveActive + return akka.persistence.typed.serialization.ReplicatedEventSourcing .internal_static_ReplicatedEventMetadata_descriptor; } @java.lang.Override protected akka.protobufv3.internal.GeneratedMessageV3.FieldAccessorTable internalGetFieldAccessorTable() { - return akka.persistence.typed.serialization.ActiveActive + return akka.persistence.typed.serialization.ReplicatedEventSourcing .internal_static_ReplicatedEventMetadata_fieldAccessorTable .ensureFieldAccessorsInitialized( - akka.persistence.typed.serialization.ActiveActive.ReplicatedEventMetadata.class, - akka.persistence.typed.serialization.ActiveActive.ReplicatedEventMetadata.Builder - .class); + akka.persistence.typed.serialization.ReplicatedEventSourcing.ReplicatedEventMetadata + .class, + akka.persistence.typed.serialization.ReplicatedEventSourcing.ReplicatedEventMetadata + .Builder.class); } private int bitField0_; @@ -6902,7 +7127,8 @@ public final class ActiveActive { } public static final int VERSIONVECTOR_FIELD_NUMBER = 3; - private akka.persistence.typed.serialization.ActiveActive.VersionVector versionVector_; + private akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector + versionVector_; /** * required .VersionVector versionVector = 3; * @@ -6916,16 +7142,19 @@ public final class ActiveActive { * * @return The versionVector. */ - public akka.persistence.typed.serialization.ActiveActive.VersionVector getVersionVector() { + public akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector + getVersionVector() { return versionVector_ == null - ? akka.persistence.typed.serialization.ActiveActive.VersionVector.getDefaultInstance() + ? akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector + .getDefaultInstance() : versionVector_; } /** required .VersionVector versionVector = 3; */ - public akka.persistence.typed.serialization.ActiveActive.VersionVectorOrBuilder + public akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVectorOrBuilder getVersionVectorOrBuilder() { return versionVector_ == null - ? akka.persistence.typed.serialization.ActiveActive.VersionVector.getDefaultInstance() + ? akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector + .getDefaultInstance() : versionVector_; } @@ -7028,11 +7257,13 @@ public final class ActiveActive { return true; } if (!(obj - instanceof akka.persistence.typed.serialization.ActiveActive.ReplicatedEventMetadata)) { + instanceof + akka.persistence.typed.serialization.ReplicatedEventSourcing.ReplicatedEventMetadata)) { return super.equals(obj); } - akka.persistence.typed.serialization.ActiveActive.ReplicatedEventMetadata other = - (akka.persistence.typed.serialization.ActiveActive.ReplicatedEventMetadata) obj; + akka.persistence.typed.serialization.ReplicatedEventSourcing.ReplicatedEventMetadata other = + (akka.persistence.typed.serialization.ReplicatedEventSourcing.ReplicatedEventMetadata) + obj; if (hasOriginReplica() != other.hasOriginReplica()) return false; if (hasOriginReplica()) { @@ -7082,13 +7313,15 @@ public final class ActiveActive { return hash; } - public static akka.persistence.typed.serialization.ActiveActive.ReplicatedEventMetadata + public static akka.persistence.typed.serialization.ReplicatedEventSourcing + .ReplicatedEventMetadata parseFrom(java.nio.ByteBuffer data) throws akka.protobufv3.internal.InvalidProtocolBufferException { return PARSER.parseFrom(data); } - public static akka.persistence.typed.serialization.ActiveActive.ReplicatedEventMetadata + public static akka.persistence.typed.serialization.ReplicatedEventSourcing + .ReplicatedEventMetadata parseFrom( java.nio.ByteBuffer data, akka.protobufv3.internal.ExtensionRegistryLite extensionRegistry) @@ -7096,13 +7329,15 @@ public final class ActiveActive { return PARSER.parseFrom(data, extensionRegistry); } - public static akka.persistence.typed.serialization.ActiveActive.ReplicatedEventMetadata + public static akka.persistence.typed.serialization.ReplicatedEventSourcing + .ReplicatedEventMetadata parseFrom(akka.protobufv3.internal.ByteString data) throws akka.protobufv3.internal.InvalidProtocolBufferException { return PARSER.parseFrom(data); } - public static akka.persistence.typed.serialization.ActiveActive.ReplicatedEventMetadata + public static akka.persistence.typed.serialization.ReplicatedEventSourcing + .ReplicatedEventMetadata parseFrom( akka.protobufv3.internal.ByteString data, akka.protobufv3.internal.ExtensionRegistryLite extensionRegistry) @@ -7110,23 +7345,27 @@ public final class ActiveActive { return PARSER.parseFrom(data, extensionRegistry); } - public static akka.persistence.typed.serialization.ActiveActive.ReplicatedEventMetadata + public static akka.persistence.typed.serialization.ReplicatedEventSourcing + .ReplicatedEventMetadata parseFrom(byte[] data) throws akka.protobufv3.internal.InvalidProtocolBufferException { return PARSER.parseFrom(data); } - public static akka.persistence.typed.serialization.ActiveActive.ReplicatedEventMetadata + public static akka.persistence.typed.serialization.ReplicatedEventSourcing + .ReplicatedEventMetadata parseFrom(byte[] data, akka.protobufv3.internal.ExtensionRegistryLite extensionRegistry) throws akka.protobufv3.internal.InvalidProtocolBufferException { return PARSER.parseFrom(data, extensionRegistry); } - public static akka.persistence.typed.serialization.ActiveActive.ReplicatedEventMetadata + public static akka.persistence.typed.serialization.ReplicatedEventSourcing + .ReplicatedEventMetadata parseFrom(java.io.InputStream input) throws java.io.IOException { return akka.protobufv3.internal.GeneratedMessageV3.parseWithIOException(PARSER, input); } - public static akka.persistence.typed.serialization.ActiveActive.ReplicatedEventMetadata + public static akka.persistence.typed.serialization.ReplicatedEventSourcing + .ReplicatedEventMetadata parseFrom( java.io.InputStream input, akka.protobufv3.internal.ExtensionRegistryLite extensionRegistry) @@ -7135,13 +7374,15 @@ public final class ActiveActive { PARSER, input, extensionRegistry); } - public static akka.persistence.typed.serialization.ActiveActive.ReplicatedEventMetadata + public static akka.persistence.typed.serialization.ReplicatedEventSourcing + .ReplicatedEventMetadata parseDelimitedFrom(java.io.InputStream input) throws java.io.IOException { return akka.protobufv3.internal.GeneratedMessageV3.parseDelimitedWithIOException( PARSER, input); } - public static akka.persistence.typed.serialization.ActiveActive.ReplicatedEventMetadata + public static akka.persistence.typed.serialization.ReplicatedEventSourcing + .ReplicatedEventMetadata parseDelimitedFrom( java.io.InputStream input, akka.protobufv3.internal.ExtensionRegistryLite extensionRegistry) @@ -7150,12 +7391,14 @@ public final class ActiveActive { PARSER, input, extensionRegistry); } - public static akka.persistence.typed.serialization.ActiveActive.ReplicatedEventMetadata + public static akka.persistence.typed.serialization.ReplicatedEventSourcing + .ReplicatedEventMetadata parseFrom(akka.protobufv3.internal.CodedInputStream input) throws java.io.IOException { return akka.protobufv3.internal.GeneratedMessageV3.parseWithIOException(PARSER, input); } - public static akka.persistence.typed.serialization.ActiveActive.ReplicatedEventMetadata + public static akka.persistence.typed.serialization.ReplicatedEventSourcing + .ReplicatedEventMetadata parseFrom( akka.protobufv3.internal.CodedInputStream input, akka.protobufv3.internal.ExtensionRegistryLite extensionRegistry) @@ -7174,7 +7417,8 @@ public final class ActiveActive { } public static Builder newBuilder( - akka.persistence.typed.serialization.ActiveActive.ReplicatedEventMetadata prototype) { + akka.persistence.typed.serialization.ReplicatedEventSourcing.ReplicatedEventMetadata + prototype) { return DEFAULT_INSTANCE.toBuilder().mergeFrom(prototype); } @@ -7194,25 +7438,27 @@ public final class ActiveActive { extends akka.protobufv3.internal.GeneratedMessageV3.Builder implements // @@protoc_insertion_point(builder_implements:ReplicatedEventMetadata) - akka.persistence.typed.serialization.ActiveActive.ReplicatedEventMetadataOrBuilder { + akka.persistence.typed.serialization.ReplicatedEventSourcing + .ReplicatedEventMetadataOrBuilder { public static final akka.protobufv3.internal.Descriptors.Descriptor getDescriptor() { - return akka.persistence.typed.serialization.ActiveActive + return akka.persistence.typed.serialization.ReplicatedEventSourcing .internal_static_ReplicatedEventMetadata_descriptor; } @java.lang.Override protected akka.protobufv3.internal.GeneratedMessageV3.FieldAccessorTable internalGetFieldAccessorTable() { - return akka.persistence.typed.serialization.ActiveActive + return akka.persistence.typed.serialization.ReplicatedEventSourcing .internal_static_ReplicatedEventMetadata_fieldAccessorTable .ensureFieldAccessorsInitialized( - akka.persistence.typed.serialization.ActiveActive.ReplicatedEventMetadata.class, - akka.persistence.typed.serialization.ActiveActive.ReplicatedEventMetadata.Builder - .class); + akka.persistence.typed.serialization.ReplicatedEventSourcing.ReplicatedEventMetadata + .class, + akka.persistence.typed.serialization.ReplicatedEventSourcing.ReplicatedEventMetadata + .Builder.class); } // Construct using - // akka.persistence.typed.serialization.ActiveActive.ReplicatedEventMetadata.newBuilder() + // akka.persistence.typed.serialization.ReplicatedEventSourcing.ReplicatedEventMetadata.newBuilder() private Builder() { maybeForceBuilderInitialization(); } @@ -7248,21 +7494,22 @@ public final class ActiveActive { @java.lang.Override public akka.protobufv3.internal.Descriptors.Descriptor getDescriptorForType() { - return akka.persistence.typed.serialization.ActiveActive + return akka.persistence.typed.serialization.ReplicatedEventSourcing .internal_static_ReplicatedEventMetadata_descriptor; } @java.lang.Override - public akka.persistence.typed.serialization.ActiveActive.ReplicatedEventMetadata + public akka.persistence.typed.serialization.ReplicatedEventSourcing.ReplicatedEventMetadata getDefaultInstanceForType() { - return akka.persistence.typed.serialization.ActiveActive.ReplicatedEventMetadata + return akka.persistence.typed.serialization.ReplicatedEventSourcing.ReplicatedEventMetadata .getDefaultInstance(); } @java.lang.Override - public akka.persistence.typed.serialization.ActiveActive.ReplicatedEventMetadata build() { - akka.persistence.typed.serialization.ActiveActive.ReplicatedEventMetadata result = - buildPartial(); + public akka.persistence.typed.serialization.ReplicatedEventSourcing.ReplicatedEventMetadata + build() { + akka.persistence.typed.serialization.ReplicatedEventSourcing.ReplicatedEventMetadata + result = buildPartial(); if (!result.isInitialized()) { throw newUninitializedMessageException(result); } @@ -7270,10 +7517,12 @@ public final class ActiveActive { } @java.lang.Override - public akka.persistence.typed.serialization.ActiveActive.ReplicatedEventMetadata + public akka.persistence.typed.serialization.ReplicatedEventSourcing.ReplicatedEventMetadata buildPartial() { - akka.persistence.typed.serialization.ActiveActive.ReplicatedEventMetadata result = - new akka.persistence.typed.serialization.ActiveActive.ReplicatedEventMetadata(this); + akka.persistence.typed.serialization.ReplicatedEventSourcing.ReplicatedEventMetadata + result = + new akka.persistence.typed.serialization.ReplicatedEventSourcing + .ReplicatedEventMetadata(this); int from_bitField0_ = bitField0_; int to_bitField0_ = 0; if (((from_bitField0_ & 0x00000001) != 0)) { @@ -7339,9 +7588,11 @@ public final class ActiveActive { @java.lang.Override public Builder mergeFrom(akka.protobufv3.internal.Message other) { if (other - instanceof akka.persistence.typed.serialization.ActiveActive.ReplicatedEventMetadata) { + instanceof + akka.persistence.typed.serialization.ReplicatedEventSourcing.ReplicatedEventMetadata) { return mergeFrom( - (akka.persistence.typed.serialization.ActiveActive.ReplicatedEventMetadata) other); + (akka.persistence.typed.serialization.ReplicatedEventSourcing.ReplicatedEventMetadata) + other); } else { super.mergeFrom(other); return this; @@ -7349,9 +7600,10 @@ public final class ActiveActive { } public Builder mergeFrom( - akka.persistence.typed.serialization.ActiveActive.ReplicatedEventMetadata other) { + akka.persistence.typed.serialization.ReplicatedEventSourcing.ReplicatedEventMetadata + other) { if (other - == akka.persistence.typed.serialization.ActiveActive.ReplicatedEventMetadata + == akka.persistence.typed.serialization.ReplicatedEventSourcing.ReplicatedEventMetadata .getDefaultInstance()) return this; if (other.hasOriginReplica()) { bitField0_ |= 0x00000001; @@ -7397,13 +7649,13 @@ public final class ActiveActive { akka.protobufv3.internal.CodedInputStream input, akka.protobufv3.internal.ExtensionRegistryLite extensionRegistry) throws java.io.IOException { - akka.persistence.typed.serialization.ActiveActive.ReplicatedEventMetadata parsedMessage = - null; + akka.persistence.typed.serialization.ReplicatedEventSourcing.ReplicatedEventMetadata + parsedMessage = null; try { parsedMessage = PARSER.parsePartialFrom(input, extensionRegistry); } catch (akka.protobufv3.internal.InvalidProtocolBufferException e) { parsedMessage = - (akka.persistence.typed.serialization.ActiveActive.ReplicatedEventMetadata) + (akka.persistence.typed.serialization.ReplicatedEventSourcing.ReplicatedEventMetadata) e.getUnfinishedMessage(); throw e.unwrapIOException(); } finally { @@ -7542,11 +7794,12 @@ public final class ActiveActive { return this; } - private akka.persistence.typed.serialization.ActiveActive.VersionVector versionVector_; + private akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector + versionVector_; private akka.protobufv3.internal.SingleFieldBuilderV3< - akka.persistence.typed.serialization.ActiveActive.VersionVector, - akka.persistence.typed.serialization.ActiveActive.VersionVector.Builder, - akka.persistence.typed.serialization.ActiveActive.VersionVectorOrBuilder> + akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector, + akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector.Builder, + akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVectorOrBuilder> versionVectorBuilder_; /** * required .VersionVector versionVector = 3; @@ -7561,10 +7814,12 @@ public final class ActiveActive { * * @return The versionVector. */ - public akka.persistence.typed.serialization.ActiveActive.VersionVector getVersionVector() { + public akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector + getVersionVector() { if (versionVectorBuilder_ == null) { return versionVector_ == null - ? akka.persistence.typed.serialization.ActiveActive.VersionVector.getDefaultInstance() + ? akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector + .getDefaultInstance() : versionVector_; } else { return versionVectorBuilder_.getMessage(); @@ -7572,7 +7827,7 @@ public final class ActiveActive { } /** required .VersionVector versionVector = 3; */ public Builder setVersionVector( - akka.persistence.typed.serialization.ActiveActive.VersionVector value) { + akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector value) { if (versionVectorBuilder_ == null) { if (value == null) { throw new NullPointerException(); @@ -7587,7 +7842,8 @@ public final class ActiveActive { } /** required .VersionVector versionVector = 3; */ public Builder setVersionVector( - akka.persistence.typed.serialization.ActiveActive.VersionVector.Builder builderForValue) { + akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector.Builder + builderForValue) { if (versionVectorBuilder_ == null) { versionVector_ = builderForValue.build(); onChanged(); @@ -7599,16 +7855,16 @@ public final class ActiveActive { } /** required .VersionVector versionVector = 3; */ public Builder mergeVersionVector( - akka.persistence.typed.serialization.ActiveActive.VersionVector value) { + akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector value) { if (versionVectorBuilder_ == null) { if (((bitField0_ & 0x00000004) != 0) && versionVector_ != null && versionVector_ - != akka.persistence.typed.serialization.ActiveActive.VersionVector + != akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector .getDefaultInstance()) { versionVector_ = - akka.persistence.typed.serialization.ActiveActive.VersionVector.newBuilder( - versionVector_) + akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector + .newBuilder(versionVector_) .mergeFrom(value) .buildPartial(); } else { @@ -7633,35 +7889,38 @@ public final class ActiveActive { return this; } /** required .VersionVector versionVector = 3; */ - public akka.persistence.typed.serialization.ActiveActive.VersionVector.Builder + public akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector.Builder getVersionVectorBuilder() { bitField0_ |= 0x00000004; onChanged(); return getVersionVectorFieldBuilder().getBuilder(); } /** required .VersionVector versionVector = 3; */ - public akka.persistence.typed.serialization.ActiveActive.VersionVectorOrBuilder + public akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVectorOrBuilder getVersionVectorOrBuilder() { if (versionVectorBuilder_ != null) { return versionVectorBuilder_.getMessageOrBuilder(); } else { return versionVector_ == null - ? akka.persistence.typed.serialization.ActiveActive.VersionVector.getDefaultInstance() + ? akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector + .getDefaultInstance() : versionVector_; } } /** required .VersionVector versionVector = 3; */ private akka.protobufv3.internal.SingleFieldBuilderV3< - akka.persistence.typed.serialization.ActiveActive.VersionVector, - akka.persistence.typed.serialization.ActiveActive.VersionVector.Builder, - akka.persistence.typed.serialization.ActiveActive.VersionVectorOrBuilder> + akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector, + akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector.Builder, + akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVectorOrBuilder> getVersionVectorFieldBuilder() { if (versionVectorBuilder_ == null) { versionVectorBuilder_ = new akka.protobufv3.internal.SingleFieldBuilderV3< - akka.persistence.typed.serialization.ActiveActive.VersionVector, - akka.persistence.typed.serialization.ActiveActive.VersionVector.Builder, - akka.persistence.typed.serialization.ActiveActive.VersionVectorOrBuilder>( + akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector, + akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector + .Builder, + akka.persistence.typed.serialization.ReplicatedEventSourcing + .VersionVectorOrBuilder>( getVersionVector(), getParentForChildren(), isClean()); versionVector_ = null; } @@ -7725,15 +7984,18 @@ public final class ActiveActive { } // @@protoc_insertion_point(class_scope:ReplicatedEventMetadata) - private static final akka.persistence.typed.serialization.ActiveActive.ReplicatedEventMetadata + private static final akka.persistence.typed.serialization.ReplicatedEventSourcing + .ReplicatedEventMetadata DEFAULT_INSTANCE; static { DEFAULT_INSTANCE = - new akka.persistence.typed.serialization.ActiveActive.ReplicatedEventMetadata(); + new akka.persistence.typed.serialization.ReplicatedEventSourcing + .ReplicatedEventMetadata(); } - public static akka.persistence.typed.serialization.ActiveActive.ReplicatedEventMetadata + public static akka.persistence.typed.serialization.ReplicatedEventSourcing + .ReplicatedEventMetadata getDefaultInstance() { return DEFAULT_INSTANCE; } @@ -7760,7 +8022,7 @@ public final class ActiveActive { } @java.lang.Override - public akka.persistence.typed.serialization.ActiveActive.ReplicatedEventMetadata + public akka.persistence.typed.serialization.ReplicatedEventSourcing.ReplicatedEventMetadata getDefaultInstanceForType() { return DEFAULT_INSTANCE; } @@ -7782,27 +8044,30 @@ public final class ActiveActive { * * @return The version. */ - akka.persistence.typed.serialization.ActiveActive.VersionVector getVersion(); + akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector getVersion(); /** required .VersionVector version = 1; */ - akka.persistence.typed.serialization.ActiveActive.VersionVectorOrBuilder getVersionOrBuilder(); + akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVectorOrBuilder + getVersionOrBuilder(); /** repeated .ReplicatedSnapshotMetadata.Seen seenPerReplica = 2; */ java.util.List< - akka.persistence.typed.serialization.ActiveActive.ReplicatedSnapshotMetadata.Seen> + akka.persistence.typed.serialization.ReplicatedEventSourcing.ReplicatedSnapshotMetadata + .Seen> getSeenPerReplicaList(); /** repeated .ReplicatedSnapshotMetadata.Seen seenPerReplica = 2; */ - akka.persistence.typed.serialization.ActiveActive.ReplicatedSnapshotMetadata.Seen + akka.persistence.typed.serialization.ReplicatedEventSourcing.ReplicatedSnapshotMetadata.Seen getSeenPerReplica(int index); /** repeated .ReplicatedSnapshotMetadata.Seen seenPerReplica = 2; */ int getSeenPerReplicaCount(); /** repeated .ReplicatedSnapshotMetadata.Seen seenPerReplica = 2; */ java.util.List< ? extends - akka.persistence.typed.serialization.ActiveActive.ReplicatedSnapshotMetadata - .SeenOrBuilder> + akka.persistence.typed.serialization.ReplicatedEventSourcing + .ReplicatedSnapshotMetadata.SeenOrBuilder> getSeenPerReplicaOrBuilderList(); /** repeated .ReplicatedSnapshotMetadata.Seen seenPerReplica = 2; */ - akka.persistence.typed.serialization.ActiveActive.ReplicatedSnapshotMetadata.SeenOrBuilder + akka.persistence.typed.serialization.ReplicatedEventSourcing.ReplicatedSnapshotMetadata + .SeenOrBuilder getSeenPerReplicaOrBuilder(int index); } /** Protobuf type {@code ReplicatedSnapshotMetadata} */ @@ -7855,14 +8120,15 @@ public final class ActiveActive { break; case 10: { - akka.persistence.typed.serialization.ActiveActive.VersionVector.Builder subBuilder = - null; + akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector.Builder + subBuilder = null; if (((bitField0_ & 0x00000001) != 0)) { subBuilder = version_.toBuilder(); } version_ = input.readMessage( - akka.persistence.typed.serialization.ActiveActive.VersionVector.PARSER, + akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector + .PARSER, extensionRegistry); if (subBuilder != null) { subBuilder.mergeFrom(version_); @@ -7876,14 +8142,14 @@ public final class ActiveActive { if (!((mutable_bitField0_ & 0x00000002) != 0)) { seenPerReplica_ = new java.util.ArrayList< - akka.persistence.typed.serialization.ActiveActive + akka.persistence.typed.serialization.ReplicatedEventSourcing .ReplicatedSnapshotMetadata.Seen>(); mutable_bitField0_ |= 0x00000002; } seenPerReplica_.add( input.readMessage( - akka.persistence.typed.serialization.ActiveActive.ReplicatedSnapshotMetadata - .Seen.PARSER, + akka.persistence.typed.serialization.ReplicatedEventSourcing + .ReplicatedSnapshotMetadata.Seen.PARSER, extensionRegistry)); break; } @@ -7911,19 +8177,20 @@ public final class ActiveActive { } public static final akka.protobufv3.internal.Descriptors.Descriptor getDescriptor() { - return akka.persistence.typed.serialization.ActiveActive + return akka.persistence.typed.serialization.ReplicatedEventSourcing .internal_static_ReplicatedSnapshotMetadata_descriptor; } @java.lang.Override protected akka.protobufv3.internal.GeneratedMessageV3.FieldAccessorTable internalGetFieldAccessorTable() { - return akka.persistence.typed.serialization.ActiveActive + return akka.persistence.typed.serialization.ReplicatedEventSourcing .internal_static_ReplicatedSnapshotMetadata_fieldAccessorTable .ensureFieldAccessorsInitialized( - akka.persistence.typed.serialization.ActiveActive.ReplicatedSnapshotMetadata.class, - akka.persistence.typed.serialization.ActiveActive.ReplicatedSnapshotMetadata.Builder - .class); + akka.persistence.typed.serialization.ReplicatedEventSourcing + .ReplicatedSnapshotMetadata.class, + akka.persistence.typed.serialization.ReplicatedEventSourcing + .ReplicatedSnapshotMetadata.Builder.class); } public interface SeenOrBuilder @@ -8043,20 +8310,20 @@ public final class ActiveActive { } public static final akka.protobufv3.internal.Descriptors.Descriptor getDescriptor() { - return akka.persistence.typed.serialization.ActiveActive + return akka.persistence.typed.serialization.ReplicatedEventSourcing .internal_static_ReplicatedSnapshotMetadata_Seen_descriptor; } @java.lang.Override protected akka.protobufv3.internal.GeneratedMessageV3.FieldAccessorTable internalGetFieldAccessorTable() { - return akka.persistence.typed.serialization.ActiveActive + return akka.persistence.typed.serialization.ReplicatedEventSourcing .internal_static_ReplicatedSnapshotMetadata_Seen_fieldAccessorTable .ensureFieldAccessorsInitialized( - akka.persistence.typed.serialization.ActiveActive.ReplicatedSnapshotMetadata.Seen - .class, - akka.persistence.typed.serialization.ActiveActive.ReplicatedSnapshotMetadata.Seen - .Builder.class); + akka.persistence.typed.serialization.ReplicatedEventSourcing + .ReplicatedSnapshotMetadata.Seen.class, + akka.persistence.typed.serialization.ReplicatedEventSourcing + .ReplicatedSnapshotMetadata.Seen.Builder.class); } private int bitField0_; @@ -8180,11 +8447,15 @@ public final class ActiveActive { } if (!(obj instanceof - akka.persistence.typed.serialization.ActiveActive.ReplicatedSnapshotMetadata.Seen)) { + akka.persistence.typed.serialization.ReplicatedEventSourcing.ReplicatedSnapshotMetadata + .Seen)) { return super.equals(obj); } - akka.persistence.typed.serialization.ActiveActive.ReplicatedSnapshotMetadata.Seen other = - (akka.persistence.typed.serialization.ActiveActive.ReplicatedSnapshotMetadata.Seen) obj; + akka.persistence.typed.serialization.ReplicatedEventSourcing.ReplicatedSnapshotMetadata.Seen + other = + (akka.persistence.typed.serialization.ReplicatedEventSourcing + .ReplicatedSnapshotMetadata.Seen) + obj; if (hasReplicaId() != other.hasReplicaId()) return false; if (hasReplicaId()) { @@ -8218,15 +8489,15 @@ public final class ActiveActive { return hash; } - public static akka.persistence.typed.serialization.ActiveActive.ReplicatedSnapshotMetadata - .Seen + public static akka.persistence.typed.serialization.ReplicatedEventSourcing + .ReplicatedSnapshotMetadata.Seen parseFrom(java.nio.ByteBuffer data) throws akka.protobufv3.internal.InvalidProtocolBufferException { return PARSER.parseFrom(data); } - public static akka.persistence.typed.serialization.ActiveActive.ReplicatedSnapshotMetadata - .Seen + public static akka.persistence.typed.serialization.ReplicatedEventSourcing + .ReplicatedSnapshotMetadata.Seen parseFrom( java.nio.ByteBuffer data, akka.protobufv3.internal.ExtensionRegistryLite extensionRegistry) @@ -8234,15 +8505,15 @@ public final class ActiveActive { return PARSER.parseFrom(data, extensionRegistry); } - public static akka.persistence.typed.serialization.ActiveActive.ReplicatedSnapshotMetadata - .Seen + public static akka.persistence.typed.serialization.ReplicatedEventSourcing + .ReplicatedSnapshotMetadata.Seen parseFrom(akka.protobufv3.internal.ByteString data) throws akka.protobufv3.internal.InvalidProtocolBufferException { return PARSER.parseFrom(data); } - public static akka.persistence.typed.serialization.ActiveActive.ReplicatedSnapshotMetadata - .Seen + public static akka.persistence.typed.serialization.ReplicatedEventSourcing + .ReplicatedSnapshotMetadata.Seen parseFrom( akka.protobufv3.internal.ByteString data, akka.protobufv3.internal.ExtensionRegistryLite extensionRegistry) @@ -8250,27 +8521,27 @@ public final class ActiveActive { return PARSER.parseFrom(data, extensionRegistry); } - public static akka.persistence.typed.serialization.ActiveActive.ReplicatedSnapshotMetadata - .Seen + public static akka.persistence.typed.serialization.ReplicatedEventSourcing + .ReplicatedSnapshotMetadata.Seen parseFrom(byte[] data) throws akka.protobufv3.internal.InvalidProtocolBufferException { return PARSER.parseFrom(data); } - public static akka.persistence.typed.serialization.ActiveActive.ReplicatedSnapshotMetadata - .Seen + public static akka.persistence.typed.serialization.ReplicatedEventSourcing + .ReplicatedSnapshotMetadata.Seen parseFrom(byte[] data, akka.protobufv3.internal.ExtensionRegistryLite extensionRegistry) throws akka.protobufv3.internal.InvalidProtocolBufferException { return PARSER.parseFrom(data, extensionRegistry); } - public static akka.persistence.typed.serialization.ActiveActive.ReplicatedSnapshotMetadata - .Seen + public static akka.persistence.typed.serialization.ReplicatedEventSourcing + .ReplicatedSnapshotMetadata.Seen parseFrom(java.io.InputStream input) throws java.io.IOException { return akka.protobufv3.internal.GeneratedMessageV3.parseWithIOException(PARSER, input); } - public static akka.persistence.typed.serialization.ActiveActive.ReplicatedSnapshotMetadata - .Seen + public static akka.persistence.typed.serialization.ReplicatedEventSourcing + .ReplicatedSnapshotMetadata.Seen parseFrom( java.io.InputStream input, akka.protobufv3.internal.ExtensionRegistryLite extensionRegistry) @@ -8279,15 +8550,15 @@ public final class ActiveActive { PARSER, input, extensionRegistry); } - public static akka.persistence.typed.serialization.ActiveActive.ReplicatedSnapshotMetadata - .Seen + public static akka.persistence.typed.serialization.ReplicatedEventSourcing + .ReplicatedSnapshotMetadata.Seen parseDelimitedFrom(java.io.InputStream input) throws java.io.IOException { return akka.protobufv3.internal.GeneratedMessageV3.parseDelimitedWithIOException( PARSER, input); } - public static akka.persistence.typed.serialization.ActiveActive.ReplicatedSnapshotMetadata - .Seen + public static akka.persistence.typed.serialization.ReplicatedEventSourcing + .ReplicatedSnapshotMetadata.Seen parseDelimitedFrom( java.io.InputStream input, akka.protobufv3.internal.ExtensionRegistryLite extensionRegistry) @@ -8296,14 +8567,14 @@ public final class ActiveActive { PARSER, input, extensionRegistry); } - public static akka.persistence.typed.serialization.ActiveActive.ReplicatedSnapshotMetadata - .Seen + public static akka.persistence.typed.serialization.ReplicatedEventSourcing + .ReplicatedSnapshotMetadata.Seen parseFrom(akka.protobufv3.internal.CodedInputStream input) throws java.io.IOException { return akka.protobufv3.internal.GeneratedMessageV3.parseWithIOException(PARSER, input); } - public static akka.persistence.typed.serialization.ActiveActive.ReplicatedSnapshotMetadata - .Seen + public static akka.persistence.typed.serialization.ReplicatedEventSourcing + .ReplicatedSnapshotMetadata.Seen parseFrom( akka.protobufv3.internal.CodedInputStream input, akka.protobufv3.internal.ExtensionRegistryLite extensionRegistry) @@ -8322,7 +8593,8 @@ public final class ActiveActive { } public static Builder newBuilder( - akka.persistence.typed.serialization.ActiveActive.ReplicatedSnapshotMetadata.Seen + akka.persistence.typed.serialization.ReplicatedEventSourcing.ReplicatedSnapshotMetadata + .Seen prototype) { return DEFAULT_INSTANCE.toBuilder().mergeFrom(prototype); } @@ -8343,27 +8615,27 @@ public final class ActiveActive { extends akka.protobufv3.internal.GeneratedMessageV3.Builder implements // @@protoc_insertion_point(builder_implements:ReplicatedSnapshotMetadata.Seen) - akka.persistence.typed.serialization.ActiveActive.ReplicatedSnapshotMetadata + akka.persistence.typed.serialization.ReplicatedEventSourcing.ReplicatedSnapshotMetadata .SeenOrBuilder { public static final akka.protobufv3.internal.Descriptors.Descriptor getDescriptor() { - return akka.persistence.typed.serialization.ActiveActive + return akka.persistence.typed.serialization.ReplicatedEventSourcing .internal_static_ReplicatedSnapshotMetadata_Seen_descriptor; } @java.lang.Override protected akka.protobufv3.internal.GeneratedMessageV3.FieldAccessorTable internalGetFieldAccessorTable() { - return akka.persistence.typed.serialization.ActiveActive + return akka.persistence.typed.serialization.ReplicatedEventSourcing .internal_static_ReplicatedSnapshotMetadata_Seen_fieldAccessorTable .ensureFieldAccessorsInitialized( - akka.persistence.typed.serialization.ActiveActive.ReplicatedSnapshotMetadata.Seen - .class, - akka.persistence.typed.serialization.ActiveActive.ReplicatedSnapshotMetadata.Seen - .Builder.class); + akka.persistence.typed.serialization.ReplicatedEventSourcing + .ReplicatedSnapshotMetadata.Seen.class, + akka.persistence.typed.serialization.ReplicatedEventSourcing + .ReplicatedSnapshotMetadata.Seen.Builder.class); } // Construct using - // akka.persistence.typed.serialization.ActiveActive.ReplicatedSnapshotMetadata.Seen.newBuilder() + // akka.persistence.typed.serialization.ReplicatedEventSourcing.ReplicatedSnapshotMetadata.Seen.newBuilder() private Builder() { maybeForceBuilderInitialization(); } @@ -8389,22 +8661,25 @@ public final class ActiveActive { @java.lang.Override public akka.protobufv3.internal.Descriptors.Descriptor getDescriptorForType() { - return akka.persistence.typed.serialization.ActiveActive + return akka.persistence.typed.serialization.ReplicatedEventSourcing .internal_static_ReplicatedSnapshotMetadata_Seen_descriptor; } @java.lang.Override - public akka.persistence.typed.serialization.ActiveActive.ReplicatedSnapshotMetadata.Seen + public akka.persistence.typed.serialization.ReplicatedEventSourcing + .ReplicatedSnapshotMetadata.Seen getDefaultInstanceForType() { - return akka.persistence.typed.serialization.ActiveActive.ReplicatedSnapshotMetadata.Seen - .getDefaultInstance(); + return akka.persistence.typed.serialization.ReplicatedEventSourcing + .ReplicatedSnapshotMetadata.Seen.getDefaultInstance(); } @java.lang.Override - public akka.persistence.typed.serialization.ActiveActive.ReplicatedSnapshotMetadata.Seen + public akka.persistence.typed.serialization.ReplicatedEventSourcing + .ReplicatedSnapshotMetadata.Seen build() { - akka.persistence.typed.serialization.ActiveActive.ReplicatedSnapshotMetadata.Seen result = - buildPartial(); + akka.persistence.typed.serialization.ReplicatedEventSourcing.ReplicatedSnapshotMetadata + .Seen + result = buildPartial(); if (!result.isInitialized()) { throw newUninitializedMessageException(result); } @@ -8412,11 +8687,14 @@ public final class ActiveActive { } @java.lang.Override - public akka.persistence.typed.serialization.ActiveActive.ReplicatedSnapshotMetadata.Seen + public akka.persistence.typed.serialization.ReplicatedEventSourcing + .ReplicatedSnapshotMetadata.Seen buildPartial() { - akka.persistence.typed.serialization.ActiveActive.ReplicatedSnapshotMetadata.Seen result = - new akka.persistence.typed.serialization.ActiveActive.ReplicatedSnapshotMetadata.Seen( - this); + akka.persistence.typed.serialization.ReplicatedEventSourcing.ReplicatedSnapshotMetadata + .Seen + result = + new akka.persistence.typed.serialization.ReplicatedEventSourcing + .ReplicatedSnapshotMetadata.Seen(this); int from_bitField0_ = bitField0_; int to_bitField0_ = 0; if (((from_bitField0_ & 0x00000001) != 0)) { @@ -8471,9 +8749,11 @@ public final class ActiveActive { public Builder mergeFrom(akka.protobufv3.internal.Message other) { if (other instanceof - akka.persistence.typed.serialization.ActiveActive.ReplicatedSnapshotMetadata.Seen) { + akka.persistence.typed.serialization.ReplicatedEventSourcing + .ReplicatedSnapshotMetadata.Seen) { return mergeFrom( - (akka.persistence.typed.serialization.ActiveActive.ReplicatedSnapshotMetadata.Seen) + (akka.persistence.typed.serialization.ReplicatedEventSourcing + .ReplicatedSnapshotMetadata.Seen) other); } else { super.mergeFrom(other); @@ -8482,11 +8762,12 @@ public final class ActiveActive { } public Builder mergeFrom( - akka.persistence.typed.serialization.ActiveActive.ReplicatedSnapshotMetadata.Seen + akka.persistence.typed.serialization.ReplicatedEventSourcing.ReplicatedSnapshotMetadata + .Seen other) { if (other - == akka.persistence.typed.serialization.ActiveActive.ReplicatedSnapshotMetadata.Seen - .getDefaultInstance()) return this; + == akka.persistence.typed.serialization.ReplicatedEventSourcing + .ReplicatedSnapshotMetadata.Seen.getDefaultInstance()) return this; if (other.hasReplicaId()) { bitField0_ |= 0x00000001; replicaId_ = other.replicaId_; @@ -8516,13 +8797,15 @@ public final class ActiveActive { akka.protobufv3.internal.CodedInputStream input, akka.protobufv3.internal.ExtensionRegistryLite extensionRegistry) throws java.io.IOException { - akka.persistence.typed.serialization.ActiveActive.ReplicatedSnapshotMetadata.Seen + akka.persistence.typed.serialization.ReplicatedEventSourcing.ReplicatedSnapshotMetadata + .Seen parsedMessage = null; try { parsedMessage = PARSER.parsePartialFrom(input, extensionRegistry); } catch (akka.protobufv3.internal.InvalidProtocolBufferException e) { parsedMessage = - (akka.persistence.typed.serialization.ActiveActive.ReplicatedSnapshotMetadata.Seen) + (akka.persistence.typed.serialization.ReplicatedEventSourcing + .ReplicatedSnapshotMetadata.Seen) e.getUnfinishedMessage(); throw e.unwrapIOException(); } finally { @@ -8677,17 +8960,18 @@ public final class ActiveActive { } // @@protoc_insertion_point(class_scope:ReplicatedSnapshotMetadata.Seen) - private static final akka.persistence.typed.serialization.ActiveActive + private static final akka.persistence.typed.serialization.ReplicatedEventSourcing .ReplicatedSnapshotMetadata.Seen DEFAULT_INSTANCE; static { DEFAULT_INSTANCE = - new akka.persistence.typed.serialization.ActiveActive.ReplicatedSnapshotMetadata.Seen(); + new akka.persistence.typed.serialization.ReplicatedEventSourcing + .ReplicatedSnapshotMetadata.Seen(); } - public static akka.persistence.typed.serialization.ActiveActive.ReplicatedSnapshotMetadata - .Seen + public static akka.persistence.typed.serialization.ReplicatedEventSourcing + .ReplicatedSnapshotMetadata.Seen getDefaultInstance() { return DEFAULT_INSTANCE; } @@ -8714,7 +8998,8 @@ public final class ActiveActive { } @java.lang.Override - public akka.persistence.typed.serialization.ActiveActive.ReplicatedSnapshotMetadata.Seen + public akka.persistence.typed.serialization.ReplicatedEventSourcing.ReplicatedSnapshotMetadata + .Seen getDefaultInstanceForType() { return DEFAULT_INSTANCE; } @@ -8722,7 +9007,7 @@ public final class ActiveActive { private int bitField0_; public static final int VERSION_FIELD_NUMBER = 1; - private akka.persistence.typed.serialization.ActiveActive.VersionVector version_; + private akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector version_; /** * required .VersionVector version = 1; * @@ -8736,34 +9021,38 @@ public final class ActiveActive { * * @return The version. */ - public akka.persistence.typed.serialization.ActiveActive.VersionVector getVersion() { + public akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector getVersion() { return version_ == null - ? akka.persistence.typed.serialization.ActiveActive.VersionVector.getDefaultInstance() + ? akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector + .getDefaultInstance() : version_; } /** required .VersionVector version = 1; */ - public akka.persistence.typed.serialization.ActiveActive.VersionVectorOrBuilder + public akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVectorOrBuilder getVersionOrBuilder() { return version_ == null - ? akka.persistence.typed.serialization.ActiveActive.VersionVector.getDefaultInstance() + ? akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector + .getDefaultInstance() : version_; } public static final int SEENPERREPLICA_FIELD_NUMBER = 2; private java.util.List< - akka.persistence.typed.serialization.ActiveActive.ReplicatedSnapshotMetadata.Seen> + akka.persistence.typed.serialization.ReplicatedEventSourcing.ReplicatedSnapshotMetadata + .Seen> seenPerReplica_; /** repeated .ReplicatedSnapshotMetadata.Seen seenPerReplica = 2; */ public java.util.List< - akka.persistence.typed.serialization.ActiveActive.ReplicatedSnapshotMetadata.Seen> + akka.persistence.typed.serialization.ReplicatedEventSourcing.ReplicatedSnapshotMetadata + .Seen> getSeenPerReplicaList() { return seenPerReplica_; } /** repeated .ReplicatedSnapshotMetadata.Seen seenPerReplica = 2; */ public java.util.List< ? extends - akka.persistence.typed.serialization.ActiveActive.ReplicatedSnapshotMetadata - .SeenOrBuilder> + akka.persistence.typed.serialization.ReplicatedEventSourcing + .ReplicatedSnapshotMetadata.SeenOrBuilder> getSeenPerReplicaOrBuilderList() { return seenPerReplica_; } @@ -8772,12 +9061,13 @@ public final class ActiveActive { return seenPerReplica_.size(); } /** repeated .ReplicatedSnapshotMetadata.Seen seenPerReplica = 2; */ - public akka.persistence.typed.serialization.ActiveActive.ReplicatedSnapshotMetadata.Seen + public akka.persistence.typed.serialization.ReplicatedEventSourcing.ReplicatedSnapshotMetadata + .Seen getSeenPerReplica(int index) { return seenPerReplica_.get(index); } /** repeated .ReplicatedSnapshotMetadata.Seen seenPerReplica = 2; */ - public akka.persistence.typed.serialization.ActiveActive.ReplicatedSnapshotMetadata + public akka.persistence.typed.serialization.ReplicatedEventSourcing.ReplicatedSnapshotMetadata .SeenOrBuilder getSeenPerReplicaOrBuilder(int index) { return seenPerReplica_.get(index); @@ -8847,11 +9137,15 @@ public final class ActiveActive { } if (!(obj instanceof - akka.persistence.typed.serialization.ActiveActive.ReplicatedSnapshotMetadata)) { + akka.persistence.typed.serialization.ReplicatedEventSourcing + .ReplicatedSnapshotMetadata)) { return super.equals(obj); } - akka.persistence.typed.serialization.ActiveActive.ReplicatedSnapshotMetadata other = - (akka.persistence.typed.serialization.ActiveActive.ReplicatedSnapshotMetadata) obj; + akka.persistence.typed.serialization.ReplicatedEventSourcing.ReplicatedSnapshotMetadata + other = + (akka.persistence.typed.serialization.ReplicatedEventSourcing + .ReplicatedSnapshotMetadata) + obj; if (hasVersion() != other.hasVersion()) return false; if (hasVersion()) { @@ -8882,13 +9176,15 @@ public final class ActiveActive { return hash; } - public static akka.persistence.typed.serialization.ActiveActive.ReplicatedSnapshotMetadata + public static akka.persistence.typed.serialization.ReplicatedEventSourcing + .ReplicatedSnapshotMetadata parseFrom(java.nio.ByteBuffer data) throws akka.protobufv3.internal.InvalidProtocolBufferException { return PARSER.parseFrom(data); } - public static akka.persistence.typed.serialization.ActiveActive.ReplicatedSnapshotMetadata + public static akka.persistence.typed.serialization.ReplicatedEventSourcing + .ReplicatedSnapshotMetadata parseFrom( java.nio.ByteBuffer data, akka.protobufv3.internal.ExtensionRegistryLite extensionRegistry) @@ -8896,13 +9192,15 @@ public final class ActiveActive { return PARSER.parseFrom(data, extensionRegistry); } - public static akka.persistence.typed.serialization.ActiveActive.ReplicatedSnapshotMetadata + public static akka.persistence.typed.serialization.ReplicatedEventSourcing + .ReplicatedSnapshotMetadata parseFrom(akka.protobufv3.internal.ByteString data) throws akka.protobufv3.internal.InvalidProtocolBufferException { return PARSER.parseFrom(data); } - public static akka.persistence.typed.serialization.ActiveActive.ReplicatedSnapshotMetadata + public static akka.persistence.typed.serialization.ReplicatedEventSourcing + .ReplicatedSnapshotMetadata parseFrom( akka.protobufv3.internal.ByteString data, akka.protobufv3.internal.ExtensionRegistryLite extensionRegistry) @@ -8910,23 +9208,27 @@ public final class ActiveActive { return PARSER.parseFrom(data, extensionRegistry); } - public static akka.persistence.typed.serialization.ActiveActive.ReplicatedSnapshotMetadata + public static akka.persistence.typed.serialization.ReplicatedEventSourcing + .ReplicatedSnapshotMetadata parseFrom(byte[] data) throws akka.protobufv3.internal.InvalidProtocolBufferException { return PARSER.parseFrom(data); } - public static akka.persistence.typed.serialization.ActiveActive.ReplicatedSnapshotMetadata + public static akka.persistence.typed.serialization.ReplicatedEventSourcing + .ReplicatedSnapshotMetadata parseFrom(byte[] data, akka.protobufv3.internal.ExtensionRegistryLite extensionRegistry) throws akka.protobufv3.internal.InvalidProtocolBufferException { return PARSER.parseFrom(data, extensionRegistry); } - public static akka.persistence.typed.serialization.ActiveActive.ReplicatedSnapshotMetadata + public static akka.persistence.typed.serialization.ReplicatedEventSourcing + .ReplicatedSnapshotMetadata parseFrom(java.io.InputStream input) throws java.io.IOException { return akka.protobufv3.internal.GeneratedMessageV3.parseWithIOException(PARSER, input); } - public static akka.persistence.typed.serialization.ActiveActive.ReplicatedSnapshotMetadata + public static akka.persistence.typed.serialization.ReplicatedEventSourcing + .ReplicatedSnapshotMetadata parseFrom( java.io.InputStream input, akka.protobufv3.internal.ExtensionRegistryLite extensionRegistry) @@ -8935,13 +9237,15 @@ public final class ActiveActive { PARSER, input, extensionRegistry); } - public static akka.persistence.typed.serialization.ActiveActive.ReplicatedSnapshotMetadata + public static akka.persistence.typed.serialization.ReplicatedEventSourcing + .ReplicatedSnapshotMetadata parseDelimitedFrom(java.io.InputStream input) throws java.io.IOException { return akka.protobufv3.internal.GeneratedMessageV3.parseDelimitedWithIOException( PARSER, input); } - public static akka.persistence.typed.serialization.ActiveActive.ReplicatedSnapshotMetadata + public static akka.persistence.typed.serialization.ReplicatedEventSourcing + .ReplicatedSnapshotMetadata parseDelimitedFrom( java.io.InputStream input, akka.protobufv3.internal.ExtensionRegistryLite extensionRegistry) @@ -8950,12 +9254,14 @@ public final class ActiveActive { PARSER, input, extensionRegistry); } - public static akka.persistence.typed.serialization.ActiveActive.ReplicatedSnapshotMetadata + public static akka.persistence.typed.serialization.ReplicatedEventSourcing + .ReplicatedSnapshotMetadata parseFrom(akka.protobufv3.internal.CodedInputStream input) throws java.io.IOException { return akka.protobufv3.internal.GeneratedMessageV3.parseWithIOException(PARSER, input); } - public static akka.persistence.typed.serialization.ActiveActive.ReplicatedSnapshotMetadata + public static akka.persistence.typed.serialization.ReplicatedEventSourcing + .ReplicatedSnapshotMetadata parseFrom( akka.protobufv3.internal.CodedInputStream input, akka.protobufv3.internal.ExtensionRegistryLite extensionRegistry) @@ -8974,7 +9280,8 @@ public final class ActiveActive { } public static Builder newBuilder( - akka.persistence.typed.serialization.ActiveActive.ReplicatedSnapshotMetadata prototype) { + akka.persistence.typed.serialization.ReplicatedEventSourcing.ReplicatedSnapshotMetadata + prototype) { return DEFAULT_INSTANCE.toBuilder().mergeFrom(prototype); } @@ -8994,25 +9301,27 @@ public final class ActiveActive { extends akka.protobufv3.internal.GeneratedMessageV3.Builder implements // @@protoc_insertion_point(builder_implements:ReplicatedSnapshotMetadata) - akka.persistence.typed.serialization.ActiveActive.ReplicatedSnapshotMetadataOrBuilder { + akka.persistence.typed.serialization.ReplicatedEventSourcing + .ReplicatedSnapshotMetadataOrBuilder { public static final akka.protobufv3.internal.Descriptors.Descriptor getDescriptor() { - return akka.persistence.typed.serialization.ActiveActive + return akka.persistence.typed.serialization.ReplicatedEventSourcing .internal_static_ReplicatedSnapshotMetadata_descriptor; } @java.lang.Override protected akka.protobufv3.internal.GeneratedMessageV3.FieldAccessorTable internalGetFieldAccessorTable() { - return akka.persistence.typed.serialization.ActiveActive + return akka.persistence.typed.serialization.ReplicatedEventSourcing .internal_static_ReplicatedSnapshotMetadata_fieldAccessorTable .ensureFieldAccessorsInitialized( - akka.persistence.typed.serialization.ActiveActive.ReplicatedSnapshotMetadata.class, - akka.persistence.typed.serialization.ActiveActive.ReplicatedSnapshotMetadata.Builder - .class); + akka.persistence.typed.serialization.ReplicatedEventSourcing + .ReplicatedSnapshotMetadata.class, + akka.persistence.typed.serialization.ReplicatedEventSourcing + .ReplicatedSnapshotMetadata.Builder.class); } // Construct using - // akka.persistence.typed.serialization.ActiveActive.ReplicatedSnapshotMetadata.newBuilder() + // akka.persistence.typed.serialization.ReplicatedEventSourcing.ReplicatedSnapshotMetadata.newBuilder() private Builder() { maybeForceBuilderInitialization(); } @@ -9049,21 +9358,22 @@ public final class ActiveActive { @java.lang.Override public akka.protobufv3.internal.Descriptors.Descriptor getDescriptorForType() { - return akka.persistence.typed.serialization.ActiveActive + return akka.persistence.typed.serialization.ReplicatedEventSourcing .internal_static_ReplicatedSnapshotMetadata_descriptor; } @java.lang.Override - public akka.persistence.typed.serialization.ActiveActive.ReplicatedSnapshotMetadata + public akka.persistence.typed.serialization.ReplicatedEventSourcing.ReplicatedSnapshotMetadata getDefaultInstanceForType() { - return akka.persistence.typed.serialization.ActiveActive.ReplicatedSnapshotMetadata - .getDefaultInstance(); + return akka.persistence.typed.serialization.ReplicatedEventSourcing + .ReplicatedSnapshotMetadata.getDefaultInstance(); } @java.lang.Override - public akka.persistence.typed.serialization.ActiveActive.ReplicatedSnapshotMetadata build() { - akka.persistence.typed.serialization.ActiveActive.ReplicatedSnapshotMetadata result = - buildPartial(); + public akka.persistence.typed.serialization.ReplicatedEventSourcing.ReplicatedSnapshotMetadata + build() { + akka.persistence.typed.serialization.ReplicatedEventSourcing.ReplicatedSnapshotMetadata + result = buildPartial(); if (!result.isInitialized()) { throw newUninitializedMessageException(result); } @@ -9071,10 +9381,12 @@ public final class ActiveActive { } @java.lang.Override - public akka.persistence.typed.serialization.ActiveActive.ReplicatedSnapshotMetadata + public akka.persistence.typed.serialization.ReplicatedEventSourcing.ReplicatedSnapshotMetadata buildPartial() { - akka.persistence.typed.serialization.ActiveActive.ReplicatedSnapshotMetadata result = - new akka.persistence.typed.serialization.ActiveActive.ReplicatedSnapshotMetadata(this); + akka.persistence.typed.serialization.ReplicatedEventSourcing.ReplicatedSnapshotMetadata + result = + new akka.persistence.typed.serialization.ReplicatedEventSourcing + .ReplicatedSnapshotMetadata(this); int from_bitField0_ = bitField0_; int to_bitField0_ = 0; if (((from_bitField0_ & 0x00000001) != 0)) { @@ -9138,9 +9450,12 @@ public final class ActiveActive { public Builder mergeFrom(akka.protobufv3.internal.Message other) { if (other instanceof - akka.persistence.typed.serialization.ActiveActive.ReplicatedSnapshotMetadata) { + akka.persistence.typed.serialization.ReplicatedEventSourcing + .ReplicatedSnapshotMetadata) { return mergeFrom( - (akka.persistence.typed.serialization.ActiveActive.ReplicatedSnapshotMetadata) other); + (akka.persistence.typed.serialization.ReplicatedEventSourcing + .ReplicatedSnapshotMetadata) + other); } else { super.mergeFrom(other); return this; @@ -9148,10 +9463,11 @@ public final class ActiveActive { } public Builder mergeFrom( - akka.persistence.typed.serialization.ActiveActive.ReplicatedSnapshotMetadata other) { + akka.persistence.typed.serialization.ReplicatedEventSourcing.ReplicatedSnapshotMetadata + other) { if (other - == akka.persistence.typed.serialization.ActiveActive.ReplicatedSnapshotMetadata - .getDefaultInstance()) return this; + == akka.persistence.typed.serialization.ReplicatedEventSourcing + .ReplicatedSnapshotMetadata.getDefaultInstance()) return this; if (other.hasVersion()) { mergeVersion(other.getVersion()); } @@ -9208,13 +9524,14 @@ public final class ActiveActive { akka.protobufv3.internal.CodedInputStream input, akka.protobufv3.internal.ExtensionRegistryLite extensionRegistry) throws java.io.IOException { - akka.persistence.typed.serialization.ActiveActive.ReplicatedSnapshotMetadata parsedMessage = - null; + akka.persistence.typed.serialization.ReplicatedEventSourcing.ReplicatedSnapshotMetadata + parsedMessage = null; try { parsedMessage = PARSER.parsePartialFrom(input, extensionRegistry); } catch (akka.protobufv3.internal.InvalidProtocolBufferException e) { parsedMessage = - (akka.persistence.typed.serialization.ActiveActive.ReplicatedSnapshotMetadata) + (akka.persistence.typed.serialization.ReplicatedEventSourcing + .ReplicatedSnapshotMetadata) e.getUnfinishedMessage(); throw e.unwrapIOException(); } finally { @@ -9227,11 +9544,11 @@ public final class ActiveActive { private int bitField0_; - private akka.persistence.typed.serialization.ActiveActive.VersionVector version_; + private akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector version_; private akka.protobufv3.internal.SingleFieldBuilderV3< - akka.persistence.typed.serialization.ActiveActive.VersionVector, - akka.persistence.typed.serialization.ActiveActive.VersionVector.Builder, - akka.persistence.typed.serialization.ActiveActive.VersionVectorOrBuilder> + akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector, + akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector.Builder, + akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVectorOrBuilder> versionBuilder_; /** * required .VersionVector version = 1; @@ -9246,10 +9563,12 @@ public final class ActiveActive { * * @return The version. */ - public akka.persistence.typed.serialization.ActiveActive.VersionVector getVersion() { + public akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector + getVersion() { if (versionBuilder_ == null) { return version_ == null - ? akka.persistence.typed.serialization.ActiveActive.VersionVector.getDefaultInstance() + ? akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector + .getDefaultInstance() : version_; } else { return versionBuilder_.getMessage(); @@ -9257,7 +9576,7 @@ public final class ActiveActive { } /** required .VersionVector version = 1; */ public Builder setVersion( - akka.persistence.typed.serialization.ActiveActive.VersionVector value) { + akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector value) { if (versionBuilder_ == null) { if (value == null) { throw new NullPointerException(); @@ -9272,7 +9591,8 @@ public final class ActiveActive { } /** required .VersionVector version = 1; */ public Builder setVersion( - akka.persistence.typed.serialization.ActiveActive.VersionVector.Builder builderForValue) { + akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector.Builder + builderForValue) { if (versionBuilder_ == null) { version_ = builderForValue.build(); onChanged(); @@ -9284,15 +9604,16 @@ public final class ActiveActive { } /** required .VersionVector version = 1; */ public Builder mergeVersion( - akka.persistence.typed.serialization.ActiveActive.VersionVector value) { + akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector value) { if (versionBuilder_ == null) { if (((bitField0_ & 0x00000001) != 0) && version_ != null && version_ - != akka.persistence.typed.serialization.ActiveActive.VersionVector + != akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector .getDefaultInstance()) { version_ = - akka.persistence.typed.serialization.ActiveActive.VersionVector.newBuilder(version_) + akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector + .newBuilder(version_) .mergeFrom(value) .buildPartial(); } else { @@ -9317,66 +9638,71 @@ public final class ActiveActive { return this; } /** required .VersionVector version = 1; */ - public akka.persistence.typed.serialization.ActiveActive.VersionVector.Builder + public akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector.Builder getVersionBuilder() { bitField0_ |= 0x00000001; onChanged(); return getVersionFieldBuilder().getBuilder(); } /** required .VersionVector version = 1; */ - public akka.persistence.typed.serialization.ActiveActive.VersionVectorOrBuilder + public akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVectorOrBuilder getVersionOrBuilder() { if (versionBuilder_ != null) { return versionBuilder_.getMessageOrBuilder(); } else { return version_ == null - ? akka.persistence.typed.serialization.ActiveActive.VersionVector.getDefaultInstance() + ? akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector + .getDefaultInstance() : version_; } } /** required .VersionVector version = 1; */ private akka.protobufv3.internal.SingleFieldBuilderV3< - akka.persistence.typed.serialization.ActiveActive.VersionVector, - akka.persistence.typed.serialization.ActiveActive.VersionVector.Builder, - akka.persistence.typed.serialization.ActiveActive.VersionVectorOrBuilder> + akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector, + akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector.Builder, + akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVectorOrBuilder> getVersionFieldBuilder() { if (versionBuilder_ == null) { versionBuilder_ = new akka.protobufv3.internal.SingleFieldBuilderV3< - akka.persistence.typed.serialization.ActiveActive.VersionVector, - akka.persistence.typed.serialization.ActiveActive.VersionVector.Builder, - akka.persistence.typed.serialization.ActiveActive.VersionVectorOrBuilder>( - getVersion(), getParentForChildren(), isClean()); + akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector, + akka.persistence.typed.serialization.ReplicatedEventSourcing.VersionVector + .Builder, + akka.persistence.typed.serialization.ReplicatedEventSourcing + .VersionVectorOrBuilder>(getVersion(), getParentForChildren(), isClean()); version_ = null; } return versionBuilder_; } private java.util.List< - akka.persistence.typed.serialization.ActiveActive.ReplicatedSnapshotMetadata.Seen> + akka.persistence.typed.serialization.ReplicatedEventSourcing + .ReplicatedSnapshotMetadata.Seen> seenPerReplica_ = java.util.Collections.emptyList(); private void ensureSeenPerReplicaIsMutable() { if (!((bitField0_ & 0x00000002) != 0)) { seenPerReplica_ = new java.util.ArrayList< - akka.persistence.typed.serialization.ActiveActive.ReplicatedSnapshotMetadata - .Seen>(seenPerReplica_); + akka.persistence.typed.serialization.ReplicatedEventSourcing + .ReplicatedSnapshotMetadata.Seen>(seenPerReplica_); bitField0_ |= 0x00000002; } } private akka.protobufv3.internal.RepeatedFieldBuilderV3< - akka.persistence.typed.serialization.ActiveActive.ReplicatedSnapshotMetadata.Seen, - akka.persistence.typed.serialization.ActiveActive.ReplicatedSnapshotMetadata.Seen - .Builder, - akka.persistence.typed.serialization.ActiveActive.ReplicatedSnapshotMetadata - .SeenOrBuilder> + akka.persistence.typed.serialization.ReplicatedEventSourcing + .ReplicatedSnapshotMetadata.Seen, + akka.persistence.typed.serialization.ReplicatedEventSourcing + .ReplicatedSnapshotMetadata.Seen.Builder, + akka.persistence.typed.serialization.ReplicatedEventSourcing + .ReplicatedSnapshotMetadata.SeenOrBuilder> seenPerReplicaBuilder_; /** repeated .ReplicatedSnapshotMetadata.Seen seenPerReplica = 2; */ public java.util.List< - akka.persistence.typed.serialization.ActiveActive.ReplicatedSnapshotMetadata.Seen> + akka.persistence.typed.serialization.ReplicatedEventSourcing + .ReplicatedSnapshotMetadata.Seen> getSeenPerReplicaList() { if (seenPerReplicaBuilder_ == null) { return java.util.Collections.unmodifiableList(seenPerReplica_); @@ -9393,7 +9719,8 @@ public final class ActiveActive { } } /** repeated .ReplicatedSnapshotMetadata.Seen seenPerReplica = 2; */ - public akka.persistence.typed.serialization.ActiveActive.ReplicatedSnapshotMetadata.Seen + public akka.persistence.typed.serialization.ReplicatedEventSourcing.ReplicatedSnapshotMetadata + .Seen getSeenPerReplica(int index) { if (seenPerReplicaBuilder_ == null) { return seenPerReplica_.get(index); @@ -9404,7 +9731,9 @@ public final class ActiveActive { /** repeated .ReplicatedSnapshotMetadata.Seen seenPerReplica = 2; */ public Builder setSeenPerReplica( int index, - akka.persistence.typed.serialization.ActiveActive.ReplicatedSnapshotMetadata.Seen value) { + akka.persistence.typed.serialization.ReplicatedEventSourcing.ReplicatedSnapshotMetadata + .Seen + value) { if (seenPerReplicaBuilder_ == null) { if (value == null) { throw new NullPointerException(); @@ -9420,7 +9749,8 @@ public final class ActiveActive { /** repeated .ReplicatedSnapshotMetadata.Seen seenPerReplica = 2; */ public Builder setSeenPerReplica( int index, - akka.persistence.typed.serialization.ActiveActive.ReplicatedSnapshotMetadata.Seen.Builder + akka.persistence.typed.serialization.ReplicatedEventSourcing.ReplicatedSnapshotMetadata + .Seen.Builder builderForValue) { if (seenPerReplicaBuilder_ == null) { ensureSeenPerReplicaIsMutable(); @@ -9433,7 +9763,9 @@ public final class ActiveActive { } /** repeated .ReplicatedSnapshotMetadata.Seen seenPerReplica = 2; */ public Builder addSeenPerReplica( - akka.persistence.typed.serialization.ActiveActive.ReplicatedSnapshotMetadata.Seen value) { + akka.persistence.typed.serialization.ReplicatedEventSourcing.ReplicatedSnapshotMetadata + .Seen + value) { if (seenPerReplicaBuilder_ == null) { if (value == null) { throw new NullPointerException(); @@ -9449,7 +9781,9 @@ public final class ActiveActive { /** repeated .ReplicatedSnapshotMetadata.Seen seenPerReplica = 2; */ public Builder addSeenPerReplica( int index, - akka.persistence.typed.serialization.ActiveActive.ReplicatedSnapshotMetadata.Seen value) { + akka.persistence.typed.serialization.ReplicatedEventSourcing.ReplicatedSnapshotMetadata + .Seen + value) { if (seenPerReplicaBuilder_ == null) { if (value == null) { throw new NullPointerException(); @@ -9464,7 +9798,8 @@ public final class ActiveActive { } /** repeated .ReplicatedSnapshotMetadata.Seen seenPerReplica = 2; */ public Builder addSeenPerReplica( - akka.persistence.typed.serialization.ActiveActive.ReplicatedSnapshotMetadata.Seen.Builder + akka.persistence.typed.serialization.ReplicatedEventSourcing.ReplicatedSnapshotMetadata + .Seen.Builder builderForValue) { if (seenPerReplicaBuilder_ == null) { ensureSeenPerReplicaIsMutable(); @@ -9478,7 +9813,8 @@ public final class ActiveActive { /** repeated .ReplicatedSnapshotMetadata.Seen seenPerReplica = 2; */ public Builder addSeenPerReplica( int index, - akka.persistence.typed.serialization.ActiveActive.ReplicatedSnapshotMetadata.Seen.Builder + akka.persistence.typed.serialization.ReplicatedEventSourcing.ReplicatedSnapshotMetadata + .Seen.Builder builderForValue) { if (seenPerReplicaBuilder_ == null) { ensureSeenPerReplicaIsMutable(); @@ -9493,8 +9829,8 @@ public final class ActiveActive { public Builder addAllSeenPerReplica( java.lang.Iterable< ? extends - akka.persistence.typed.serialization.ActiveActive.ReplicatedSnapshotMetadata - .Seen> + akka.persistence.typed.serialization.ReplicatedEventSourcing + .ReplicatedSnapshotMetadata.Seen> values) { if (seenPerReplicaBuilder_ == null) { ensureSeenPerReplicaIsMutable(); @@ -9528,13 +9864,13 @@ public final class ActiveActive { return this; } /** repeated .ReplicatedSnapshotMetadata.Seen seenPerReplica = 2; */ - public akka.persistence.typed.serialization.ActiveActive.ReplicatedSnapshotMetadata.Seen - .Builder + public akka.persistence.typed.serialization.ReplicatedEventSourcing.ReplicatedSnapshotMetadata + .Seen.Builder getSeenPerReplicaBuilder(int index) { return getSeenPerReplicaFieldBuilder().getBuilder(index); } /** repeated .ReplicatedSnapshotMetadata.Seen seenPerReplica = 2; */ - public akka.persistence.typed.serialization.ActiveActive.ReplicatedSnapshotMetadata + public akka.persistence.typed.serialization.ReplicatedEventSourcing.ReplicatedSnapshotMetadata .SeenOrBuilder getSeenPerReplicaOrBuilder(int index) { if (seenPerReplicaBuilder_ == null) { @@ -9546,8 +9882,8 @@ public final class ActiveActive { /** repeated .ReplicatedSnapshotMetadata.Seen seenPerReplica = 2; */ public java.util.List< ? extends - akka.persistence.typed.serialization.ActiveActive.ReplicatedSnapshotMetadata - .SeenOrBuilder> + akka.persistence.typed.serialization.ReplicatedEventSourcing + .ReplicatedSnapshotMetadata.SeenOrBuilder> getSeenPerReplicaOrBuilderList() { if (seenPerReplicaBuilder_ != null) { return seenPerReplicaBuilder_.getMessageOrBuilderList(); @@ -9556,47 +9892,49 @@ public final class ActiveActive { } } /** repeated .ReplicatedSnapshotMetadata.Seen seenPerReplica = 2; */ - public akka.persistence.typed.serialization.ActiveActive.ReplicatedSnapshotMetadata.Seen - .Builder + public akka.persistence.typed.serialization.ReplicatedEventSourcing.ReplicatedSnapshotMetadata + .Seen.Builder addSeenPerReplicaBuilder() { return getSeenPerReplicaFieldBuilder() .addBuilder( - akka.persistence.typed.serialization.ActiveActive.ReplicatedSnapshotMetadata.Seen - .getDefaultInstance()); + akka.persistence.typed.serialization.ReplicatedEventSourcing + .ReplicatedSnapshotMetadata.Seen.getDefaultInstance()); } /** repeated .ReplicatedSnapshotMetadata.Seen seenPerReplica = 2; */ - public akka.persistence.typed.serialization.ActiveActive.ReplicatedSnapshotMetadata.Seen - .Builder + public akka.persistence.typed.serialization.ReplicatedEventSourcing.ReplicatedSnapshotMetadata + .Seen.Builder addSeenPerReplicaBuilder(int index) { return getSeenPerReplicaFieldBuilder() .addBuilder( index, - akka.persistence.typed.serialization.ActiveActive.ReplicatedSnapshotMetadata.Seen - .getDefaultInstance()); + akka.persistence.typed.serialization.ReplicatedEventSourcing + .ReplicatedSnapshotMetadata.Seen.getDefaultInstance()); } /** repeated .ReplicatedSnapshotMetadata.Seen seenPerReplica = 2; */ public java.util.List< - akka.persistence.typed.serialization.ActiveActive.ReplicatedSnapshotMetadata.Seen - .Builder> + akka.persistence.typed.serialization.ReplicatedEventSourcing + .ReplicatedSnapshotMetadata.Seen.Builder> getSeenPerReplicaBuilderList() { return getSeenPerReplicaFieldBuilder().getBuilderList(); } private akka.protobufv3.internal.RepeatedFieldBuilderV3< - akka.persistence.typed.serialization.ActiveActive.ReplicatedSnapshotMetadata.Seen, - akka.persistence.typed.serialization.ActiveActive.ReplicatedSnapshotMetadata.Seen - .Builder, - akka.persistence.typed.serialization.ActiveActive.ReplicatedSnapshotMetadata - .SeenOrBuilder> + akka.persistence.typed.serialization.ReplicatedEventSourcing + .ReplicatedSnapshotMetadata.Seen, + akka.persistence.typed.serialization.ReplicatedEventSourcing + .ReplicatedSnapshotMetadata.Seen.Builder, + akka.persistence.typed.serialization.ReplicatedEventSourcing + .ReplicatedSnapshotMetadata.SeenOrBuilder> getSeenPerReplicaFieldBuilder() { if (seenPerReplicaBuilder_ == null) { seenPerReplicaBuilder_ = new akka.protobufv3.internal.RepeatedFieldBuilderV3< - akka.persistence.typed.serialization.ActiveActive.ReplicatedSnapshotMetadata.Seen, - akka.persistence.typed.serialization.ActiveActive.ReplicatedSnapshotMetadata.Seen - .Builder, - akka.persistence.typed.serialization.ActiveActive.ReplicatedSnapshotMetadata - .SeenOrBuilder>( + akka.persistence.typed.serialization.ReplicatedEventSourcing + .ReplicatedSnapshotMetadata.Seen, + akka.persistence.typed.serialization.ReplicatedEventSourcing + .ReplicatedSnapshotMetadata.Seen.Builder, + akka.persistence.typed.serialization.ReplicatedEventSourcing + .ReplicatedSnapshotMetadata.SeenOrBuilder>( seenPerReplica_, ((bitField0_ & 0x00000002) != 0), getParentForChildren(), @@ -9622,16 +9960,18 @@ public final class ActiveActive { } // @@protoc_insertion_point(class_scope:ReplicatedSnapshotMetadata) - private static final akka.persistence.typed.serialization.ActiveActive + private static final akka.persistence.typed.serialization.ReplicatedEventSourcing .ReplicatedSnapshotMetadata DEFAULT_INSTANCE; static { DEFAULT_INSTANCE = - new akka.persistence.typed.serialization.ActiveActive.ReplicatedSnapshotMetadata(); + new akka.persistence.typed.serialization.ReplicatedEventSourcing + .ReplicatedSnapshotMetadata(); } - public static akka.persistence.typed.serialization.ActiveActive.ReplicatedSnapshotMetadata + public static akka.persistence.typed.serialization.ReplicatedEventSourcing + .ReplicatedSnapshotMetadata getDefaultInstance() { return DEFAULT_INSTANCE; } @@ -9658,7 +9998,7 @@ public final class ActiveActive { } @java.lang.Override - public akka.persistence.typed.serialization.ActiveActive.ReplicatedSnapshotMetadata + public akka.persistence.typed.serialization.ReplicatedEventSourcing.ReplicatedSnapshotMetadata getDefaultInstanceForType() { return DEFAULT_INSTANCE; } @@ -9713,30 +10053,30 @@ public final class ActiveActive { static { java.lang.String[] descriptorData = { - "\n\022ActiveActive.proto\032\026ContainerFormats.p" - + "roto\"\030\n\007Counter\022\r\n\005value\030\001 \002(\014\"\036\n\rCounte" - + "rUpdate\022\r\n\005delta\030\001 \002(\014\"\304\001\n\005ORSet\022\020\n\010orig" - + "inDc\030\001 \002(\t\022\037\n\007vvector\030\002 \002(\0132\016.VersionVec" - + "tor\022\034\n\004dots\030\003 \003(\0132\016.VersionVector\022\026\n\016str" - + "ingElements\030\004 \003(\t\022\027\n\013intElements\030\005 \003(\021B\002" - + "\020\001\022\030\n\014longElements\030\006 \003(\022B\002\020\001\022\037\n\rotherEle" - + "ments\030\007 \003(\0132\010.Payload\"\201\001\n\017ORSetDeltaGrou" - + "p\022\'\n\007entries\030\001 \003(\0132\026.ORSetDeltaGroup.Ent" - + "ry\032E\n\005Entry\022 \n\toperation\030\001 \002(\0162\r.ORSetDe" - + "ltaOp\022\032\n\nunderlying\030\002 \002(\0132\006.ORSet\"]\n\rVer" - + "sionVector\022%\n\007entries\030\001 \003(\0132\024.VersionVec" - + "tor.Entry\032%\n\005Entry\022\013\n\003key\030\001 \002(\t\022\017\n\007versi" - + "on\030\002 \002(\003\"\205\001\n\027ReplicatedEventMetadata\022\025\n\r" - + "originReplica\030\001 \002(\t\022\030\n\020originSequenceNr\030" - + "\002 \002(\003\022%\n\rversionVector\030\003 \002(\0132\016.VersionVe" - + "ctor\022\022\n\nconcurrent\030\004 \002(\010\"\246\001\n\032ReplicatedS" - + "napshotMetadata\022\037\n\007version\030\001 \002(\0132\016.Versi" - + "onVector\0228\n\016seenPerReplica\030\002 \003(\0132 .Repli" - + "catedSnapshotMetadata.Seen\032-\n\004Seen\022\021\n\tre" - + "plicaId\030\001 \002(\t\022\022\n\nsequenceNr\030\002 \002(\003*-\n\014ORS" - + "etDeltaOp\022\007\n\003Add\020\000\022\n\n\006Remove\020\001\022\010\n\004Full\020\002" - + "B(\n$akka.persistence.typed.serialization" - + "H\001" + "\n\035ReplicatedEventSourcing.proto\032\026Contain" + + "erFormats.proto\"\030\n\007Counter\022\r\n\005value\030\001 \002(" + + "\014\"\036\n\rCounterUpdate\022\r\n\005delta\030\001 \002(\014\"\304\001\n\005OR" + + "Set\022\020\n\010originDc\030\001 \002(\t\022\037\n\007vvector\030\002 \002(\0132\016" + + ".VersionVector\022\034\n\004dots\030\003 \003(\0132\016.VersionVe" + + "ctor\022\026\n\016stringElements\030\004 \003(\t\022\027\n\013intEleme" + + "nts\030\005 \003(\021B\002\020\001\022\030\n\014longElements\030\006 \003(\022B\002\020\001\022" + + "\037\n\rotherElements\030\007 \003(\0132\010.Payload\"\201\001\n\017ORS" + + "etDeltaGroup\022\'\n\007entries\030\001 \003(\0132\026.ORSetDel" + + "taGroup.Entry\032E\n\005Entry\022 \n\toperation\030\001 \002(" + + "\0162\r.ORSetDeltaOp\022\032\n\nunderlying\030\002 \002(\0132\006.O" + + "RSet\"]\n\rVersionVector\022%\n\007entries\030\001 \003(\0132\024" + + ".VersionVector.Entry\032%\n\005Entry\022\013\n\003key\030\001 \002" + + "(\t\022\017\n\007version\030\002 \002(\003\"\205\001\n\027ReplicatedEventM" + + "etadata\022\025\n\roriginReplica\030\001 \002(\t\022\030\n\020origin" + + "SequenceNr\030\002 \002(\003\022%\n\rversionVector\030\003 \002(\0132" + + "\016.VersionVector\022\022\n\nconcurrent\030\004 \002(\010\"\246\001\n\032" + + "ReplicatedSnapshotMetadata\022\037\n\007version\030\001 " + + "\002(\0132\016.VersionVector\0228\n\016seenPerReplica\030\002 " + + "\003(\0132 .ReplicatedSnapshotMetadata.Seen\032-\n" + + "\004Seen\022\021\n\treplicaId\030\001 \002(\t\022\022\n\nsequenceNr\030\002" + + " \002(\003*-\n\014ORSetDeltaOp\022\007\n\003Add\020\000\022\n\n\006Remove\020" + + "\001\022\010\n\004Full\020\002B(\n$akka.persistence.typed.se" + + "rializationH\001" }; descriptor = akka.protobufv3.internal.Descriptors.FileDescriptor.internalBuildGeneratedFileFrom( diff --git a/akka-persistence-typed/src/main/mima-filters/2.6.7.backwards.excludes/29217-active-active-event-sourcing.excludes b/akka-persistence-typed/src/main/mima-filters/2.6.7.backwards.excludes/29217-replicated-event-sourcing.excludes similarity index 83% rename from akka-persistence-typed/src/main/mima-filters/2.6.7.backwards.excludes/29217-active-active-event-sourcing.excludes rename to akka-persistence-typed/src/main/mima-filters/2.6.7.backwards.excludes/29217-replicated-event-sourcing.excludes index 5a6490fdc2..c0cfd45bdd 100644 --- a/akka-persistence-typed/src/main/mima-filters/2.6.7.backwards.excludes/29217-active-active-event-sourcing.excludes +++ b/akka-persistence-typed/src/main/mima-filters/2.6.7.backwards.excludes/29217-replicated-event-sourcing.excludes @@ -1,4 +1,4 @@ # Changes to internal/private/do not extend -ProblemFilters.exclude[ReversedMissingMethodProblem]("akka.persistence.typed.scaladsl.EventSourcedBehavior.withActiveActive") +ProblemFilters.exclude[ReversedMissingMethodProblem]("akka.persistence.typed.scaladsl.EventSourcedBehavior.withReplication") ProblemFilters.exclude[Problem]("akka.persistence.typed.internal.*") ProblemFilters.exclude[ReversedMissingMethodProblem]("akka.persistence.typed.scaladsl.EventSourcedBehavior.withEventPublishing") diff --git a/akka-persistence-typed/src/main/protobuf/ActiveActive.proto b/akka-persistence-typed/src/main/protobuf/ReplicatedEventSourcing.proto similarity index 100% rename from akka-persistence-typed/src/main/protobuf/ActiveActive.proto rename to akka-persistence-typed/src/main/protobuf/ReplicatedEventSourcing.proto diff --git a/akka-persistence-typed/src/main/resources/reference.conf b/akka-persistence-typed/src/main/resources/reference.conf index 3e1c20d1b3..fb72910aca 100644 --- a/akka-persistence-typed/src/main/resources/reference.conf +++ b/akka-persistence-typed/src/main/resources/reference.conf @@ -1,8 +1,8 @@ akka.actor { - serialization-identifiers."akka.persistence.typed.serialization.ActiveActiveSerializer" = 40 + serialization-identifiers."akka.persistence.typed.serialization.ReplicatedEventSourcingSerializer" = 40 - serializers.active-active = "akka.persistence.typed.serialization.ActiveActiveSerializer" + serializers.active-active = "akka.persistence.typed.serialization.ReplicatedEventSourcingSerializer" serialization-bindings { "akka.persistence.typed.internal.VersionVector" = active-active diff --git a/akka-persistence-typed/src/main/scala/akka/persistence/typed/PublishedEvent.scala b/akka-persistence-typed/src/main/scala/akka/persistence/typed/PublishedEvent.scala index 1311ea6574..1bdd6889b1 100644 --- a/akka-persistence-typed/src/main/scala/akka/persistence/typed/PublishedEvent.scala +++ b/akka-persistence-typed/src/main/scala/akka/persistence/typed/PublishedEvent.scala @@ -17,10 +17,10 @@ import akka.persistence.typed.internal.ReplicatedPublishedEventMetaData @DoNotInherit trait PublishedEvent { - /** Scala API: When emitted from an Active Active actor this will contain the replica id */ + /** Scala API: When emitted from an Replicated Event Sourcing actor this will contain the replica id */ def replicatedMetaData: Option[ReplicatedPublishedEventMetaData] - /** Java API: When emitted from an Active Active actor this will contain the replica id */ + /** Java API: When emitted from an Replicated Event Sourcing actor this will contain the replica id */ def getReplicatedMetaData: Optional[ReplicatedPublishedEventMetaData] def persistenceId: PersistenceId diff --git a/akka-persistence-typed/src/main/scala/akka/persistence/typed/ReplicaId.scala b/akka-persistence-typed/src/main/scala/akka/persistence/typed/ReplicaId.scala index b01082a980..a842bf1f35 100644 --- a/akka-persistence-typed/src/main/scala/akka/persistence/typed/ReplicaId.scala +++ b/akka-persistence-typed/src/main/scala/akka/persistence/typed/ReplicaId.scala @@ -5,6 +5,6 @@ package akka.persistence.typed /** - * Identifies a replica in Active Active eventsourcing, could be a datacenter name or a logical identifier. + * Identifies a replica in Replicated Event Sourcing, could be a datacenter name or a logical identifier. */ final case class ReplicaId(id: String) diff --git a/akka-persistence-typed/src/main/scala/akka/persistence/typed/internal/BehaviorSetup.scala b/akka-persistence-typed/src/main/scala/akka/persistence/typed/internal/BehaviorSetup.scala index 4528b602c4..fb745a89a8 100644 --- a/akka-persistence-typed/src/main/scala/akka/persistence/typed/internal/BehaviorSetup.scala +++ b/akka-persistence-typed/src/main/scala/akka/persistence/typed/internal/BehaviorSetup.scala @@ -13,7 +13,6 @@ import akka.actor.typed.scaladsl.ActorContext import akka.annotation.InternalApi import akka.persistence._ import akka.persistence.typed.ReplicaId -import akka.persistence.typed.scaladsl.EventSourcedBehavior.ActiveActive import akka.persistence.typed.{ EventAdapter, PersistenceId, SnapshotAdapter } import akka.persistence.typed.scaladsl.{ EventSourcedBehavior, RetentionCriteria } import akka.util.OptionVal @@ -49,7 +48,7 @@ private[akka] final class BehaviorSetup[C, E, S]( var holdingRecoveryPermit: Boolean, val settings: EventSourcedSettings, val stashState: StashState, - val activeActive: Option[ActiveActive], + val replication: Option[ReplicationSetup], val publishEvents: Boolean) { import BehaviorSetup._ @@ -62,7 +61,7 @@ private[akka] final class BehaviorSetup[C, E, S]( val journal: ClassicActorRef = persistence.journalFor(settings.journalPluginId) val snapshotStore: ClassicActorRef = persistence.snapshotStoreFor(settings.snapshotPluginId) - val replicaId: Option[ReplicaId] = activeActive.map(_.replicaId) + val replicaId: Option[ReplicaId] = replication.map(_.replicaId) def selfClassic: ClassicActorRef = context.self.toClassic diff --git a/akka-persistence-typed/src/main/scala/akka/persistence/typed/internal/EventSourcedBehaviorImpl.scala b/akka-persistence-typed/src/main/scala/akka/persistence/typed/internal/EventSourcedBehaviorImpl.scala index 521ba48cfb..299cfab3d5 100644 --- a/akka-persistence-typed/src/main/scala/akka/persistence/typed/internal/EventSourcedBehaviorImpl.scala +++ b/akka-persistence-typed/src/main/scala/akka/persistence/typed/internal/EventSourcedBehaviorImpl.scala @@ -40,7 +40,6 @@ import akka.persistence.typed.SnapshotAdapter import akka.persistence.typed.SnapshotCompleted import akka.persistence.typed.SnapshotFailed import akka.persistence.typed.SnapshotSelectionCriteria -import akka.persistence.typed.scaladsl.EventSourcedBehavior.ActiveActive import akka.persistence.typed.scaladsl._ import akka.persistence.typed.scaladsl.{ Recovery => TypedRecovery } import akka.persistence.typed.scaladsl.RetentionCriteria @@ -93,7 +92,7 @@ private[akka] final case class EventSourcedBehaviorImpl[Command, Event, State]( retention: RetentionCriteria = RetentionCriteria.disabled, supervisionStrategy: SupervisorStrategy = SupervisorStrategy.stop, override val signalHandler: PartialFunction[(State, Signal), Unit] = PartialFunction.empty, - activeActive: Option[ActiveActive] = None, + replication: Option[ReplicationSetup] = None, publishEvents: Boolean = false) extends EventSourcedBehavior[Command, Event, State] { @@ -158,7 +157,7 @@ private[akka] final case class EventSourcedBehaviorImpl[Command, Event, State]( holdingRecoveryPermit = false, settings = settings, stashState = stashState, - activeActive = activeActive, + replication = replication, publishEvents = publishEvents) // needs to accept Any since we also can get messages from the journal @@ -251,9 +250,9 @@ private[akka] final case class EventSourcedBehaviorImpl[Command, Event, State]( copy(publishEvents = true) } - override private[akka] def withActiveActive( - context: ActiveActiveContextImpl): EventSourcedBehavior[Command, Event, State] = { - copy(activeActive = Some(ActiveActive(context.replicaId, context.replicasAndQueryPlugins, context))) + override private[akka] def withReplication( + context: ReplicationContextImpl): EventSourcedBehavior[Command, Event, State] = { + copy(replication = Some(ReplicationSetup(context.replicaId, context.replicasAndQueryPlugins, context))) } } @@ -274,7 +273,7 @@ private[akka] final case class EventSourcedBehaviorImpl[Command, Event, State]( object ReplicatedEventMetadata { /** - * For a journal supporting active active needing to add test coverage, use this instance as metadata and defer + * For a journal supporting Replicated Event Sourcing needing to add test coverage, use this instance as metadata and defer * to the built in serializer for serialization format */ @ApiMayChange @@ -297,7 +296,7 @@ private[akka] final case class ReplicatedEventMetadata( object ReplicatedSnapshotMetadata { /** - * For a snapshot store supporting active active needing to add test coverage, use this instance as metadata and defer + * For a snapshot store supporting Replicated Event Sourcing needing to add test coverage, use this instance as metadata and defer * to the built in serializer for serialization format */ @ApiMayChange diff --git a/akka-persistence-typed/src/main/scala/akka/persistence/typed/internal/ExternalInteractions.scala b/akka-persistence-typed/src/main/scala/akka/persistence/typed/internal/ExternalInteractions.scala index 04ae7984b2..5f807e9694 100644 --- a/akka-persistence-typed/src/main/scala/akka/persistence/typed/internal/ExternalInteractions.scala +++ b/akka-persistence-typed/src/main/scala/akka/persistence/typed/internal/ExternalInteractions.scala @@ -191,7 +191,7 @@ private[akka] trait SnapshotInteractions[C, E, S] { if (state.state == null) throw new IllegalStateException("A snapshot must not be a null state.") else { - val meta = setup.activeActive match { + val meta = setup.replication match { case Some(_) => val m = ReplicatedSnapshotMetadata(state.version, state.seenPerReplica) Some(m) diff --git a/akka-persistence-typed/src/main/scala/akka/persistence/typed/internal/ReplayingEvents.scala b/akka-persistence-typed/src/main/scala/akka/persistence/typed/internal/ReplayingEvents.scala index e33300a277..992b2a1ca4 100644 --- a/akka-persistence-typed/src/main/scala/akka/persistence/typed/internal/ReplayingEvents.scala +++ b/akka-persistence-typed/src/main/scala/akka/persistence/typed/internal/ReplayingEvents.scala @@ -23,7 +23,6 @@ import akka.persistence.typed.SingleEventSeq import akka.persistence.typed.internal.EventSourcedBehaviorImpl.GetState import akka.persistence.typed.internal.ReplayingEvents.ReplayingState import akka.persistence.typed.internal.Running.WithSeqNrAccessible -import akka.persistence.typed.scaladsl.EventSourcedBehavior.ActiveActive import akka.util.OptionVal import akka.util.PrettyDuration._ import akka.util.unused @@ -123,14 +122,14 @@ private[akka] final class ReplayingEvents[C, E, S]( eventForErrorReporting = OptionVal.Some(event) state = state.copy(seqNr = repr.sequenceNr) - val aaMetaAndSelfReplica: Option[(ReplicatedEventMetadata, ReplicaId, ActiveActive)] = - setup.activeActive match { + val aaMetaAndSelfReplica: Option[(ReplicatedEventMetadata, ReplicaId, ReplicationSetup)] = + setup.replication match { case Some(aa) => val meta = repr.metadata match { case Some(m) => m.asInstanceOf[ReplicatedEventMetadata] case None => throw new IllegalStateException( - s"Active active enabled but existing event has no metadata. Migration isn't supported yet.") + s"Replicated Event Sourcing enabled but existing event has no metadata. Migration isn't supported yet.") } aa.setContext(recoveryRunning = true, meta.originReplica, meta.concurrent) @@ -140,7 +139,7 @@ private[akka] final class ReplayingEvents[C, E, S]( val newState = setup.eventHandler(state.state, event) - setup.activeActive match { + setup.replication match { case Some(aa) => aa.clearContext() case None => diff --git a/akka-persistence-typed/src/main/scala/akka/persistence/typed/internal/ReplicationSetup.scala b/akka-persistence-typed/src/main/scala/akka/persistence/typed/internal/ReplicationSetup.scala new file mode 100644 index 0000000000..755201eb63 --- /dev/null +++ b/akka-persistence-typed/src/main/scala/akka/persistence/typed/internal/ReplicationSetup.scala @@ -0,0 +1,111 @@ +/* + * Copyright (C) 2020 Lightbend Inc. + */ + +package akka.persistence.typed.internal + +import akka.annotation.InternalApi +import akka.persistence.typed.PersistenceId +import akka.persistence.typed.ReplicaId +import akka.util.OptionVal +import akka.util.WallClock +import akka.util.ccompat.JavaConverters._ + +/** + * INTERNAL API + */ +// FIXME, parts of this can be set during initialisation +// Other fields will be set before executing the event handler as they change per event +// https://github.com/akka/akka/issues/29258 +@InternalApi +private[akka] final class ReplicationContextImpl( + val entityId: String, + val replicaId: ReplicaId, + val replicasAndQueryPlugins: Map[ReplicaId, String]) + extends akka.persistence.typed.scaladsl.ReplicationContext + with akka.persistence.typed.javadsl.ReplicationContext { + val allReplicas: Set[ReplicaId] = replicasAndQueryPlugins.keySet + + // these are not volatile as they are set on the same thread as they should be accessed + var _currentThread: OptionVal[Thread] = OptionVal.None + var _origin: OptionVal[ReplicaId] = OptionVal.None + var _recoveryRunning: Boolean = false + var _concurrent: Boolean = false + + private def checkAccess(functionName: String): Unit = { + val callerThread = Thread.currentThread() + def error() = + throw new UnsupportedOperationException( + s"Unsupported access to ReplicationContext operation from the outside of event handler. " + + s"$functionName can only be called from the event handler") + _currentThread match { + case OptionVal.Some(t) => + if (callerThread ne t) error() + case OptionVal.None => + error() + } + } + + /** + * The origin of the current event. + * Undefined result if called from anywhere other than an event handler. + */ + override def origin: ReplicaId = { + checkAccess("origin") + _origin match { + case OptionVal.Some(origin) => origin + case OptionVal.None => throw new IllegalStateException("origin can only be accessed from the event handler") + } + } + + /** + * Whether the happened concurrently with an event from another replica. + * Undefined result if called from any where other than an event handler. + */ + override def concurrent: Boolean = { + checkAccess("concurrent") + _concurrent + } + + override def persistenceId: PersistenceId = PersistenceId.replicatedUniqueId(entityId, replicaId) + + override def currentTimeMillis(): Long = { + WallClock.AlwaysIncreasingClock.currentTimeMillis() + } + override def recoveryRunning: Boolean = { + checkAccess("recoveryRunning") + _recoveryRunning + } + + override def getAllReplicas: java.util.Set[ReplicaId] = allReplicas.asJava +} + +/** + * INTERNAL API + */ +@InternalApi +private[akka] final case class ReplicationSetup( + replicaId: ReplicaId, + allReplicasAndQueryPlugins: Map[ReplicaId, String], + aaContext: ReplicationContextImpl) { + + val allReplicas: Set[ReplicaId] = allReplicasAndQueryPlugins.keySet + + /** + * Must only be called on the same thread that will execute the user code + */ + def setContext(recoveryRunning: Boolean, originReplica: ReplicaId, concurrent: Boolean): Unit = { + aaContext._currentThread = OptionVal.Some(Thread.currentThread()) + aaContext._recoveryRunning = recoveryRunning + aaContext._concurrent = concurrent + aaContext._origin = OptionVal.Some(originReplica) + } + + def clearContext(): Unit = { + aaContext._currentThread = OptionVal.None + aaContext._recoveryRunning = false + aaContext._concurrent = false + aaContext._origin = OptionVal.None + } + +} diff --git a/akka-persistence-typed/src/main/scala/akka/persistence/typed/internal/Running.scala b/akka-persistence-typed/src/main/scala/akka/persistence/typed/internal/Running.scala index 4eeec5bdc1..5815e80653 100644 --- a/akka-persistence-typed/src/main/scala/akka/persistence/typed/internal/Running.scala +++ b/akka-persistence-typed/src/main/scala/akka/persistence/typed/internal/Running.scala @@ -53,7 +53,6 @@ import akka.persistence.typed.internal.InternalProtocol.ReplicatedEventEnvelope import akka.persistence.typed.internal.JournalInteractions.EventToPersist import akka.persistence.typed.internal.Running.WithSeqNrAccessible import akka.persistence.typed.scaladsl.Effect -import akka.persistence.typed.scaladsl.EventSourcedBehavior.ActiveActive import akka.stream.scaladsl.Keep import akka.stream.SystemMaterializer import akka.stream.WatchedActorTerminatedException @@ -111,7 +110,7 @@ private[akka] object Running { def apply[C, E, S](setup: BehaviorSetup[C, E, S], state: RunningState[S]): Behavior[InternalProtocol] = { val running = new Running(setup.setMdcPhase(PersistenceMdc.RunningCmds)) - val initialState = setup.activeActive match { + val initialState = setup.replication match { case Some(aa) => startReplicationStream(setup, state, aa) case None => state } @@ -121,17 +120,17 @@ private[akka] object Running { def startReplicationStream[C, E, S]( setup: BehaviorSetup[C, E, S], state: RunningState[S], - aa: ActiveActive): RunningState[S] = { + replicationSetup: ReplicationSetup): RunningState[S] = { import scala.concurrent.duration._ val system = setup.context.system val ref = setup.context.self val query = PersistenceQuery(system) - aa.allReplicas.foldLeft(state) { (state, replicaId) => - if (replicaId != aa.replicaId) { + replicationSetup.allReplicas.foldLeft(state) { (state, replicaId) => + if (replicaId != replicationSetup.replicaId) { val seqNr = state.seenPerReplica(replicaId) - val pid = PersistenceId.replicatedUniqueId(aa.aaContext.entityId, replicaId) - val queryPluginId = aa.allReplicasAndQueryPlugins(replicaId) + val pid = PersistenceId.replicatedUniqueId(replicationSetup.aaContext.entityId, replicaId) + val queryPluginId = replicationSetup.allReplicasAndQueryPlugins(replicaId) val replication = query.readJournalFor[EventsByPersistenceIdQuery](queryPluginId) implicit val timeout = Timeout(30.seconds) @@ -152,7 +151,7 @@ private[akka] object Running { s"Replication stream from replica ${replicaId} for ${setup.persistenceId} contains event " + s"(sequence nr ${event.sequenceNr}) without replication metadata. " + s"Is the persistence id used by a regular event sourced actor there or the journal for that replica (${queryPluginId}) " + - "used that does not support active active?") + "used that does not support Replicated Event Sourcing?") }) .viaMat(new FastForwardingFilter)(Keep.right) .mapMaterializedValue(streamControl => controlRef.set(streamControl)) @@ -240,7 +239,7 @@ private[akka] object Running { def onMessage(msg: InternalProtocol): Behavior[InternalProtocol] = msg match { case IncomingCommand(c: C @unchecked) => onCommand(state, c) - case re: ReplicatedEventEnvelope[E @unchecked] => onReplicatedEvent(state, re, setup.activeActive.get) + case re: ReplicatedEventEnvelope[E @unchecked] => onReplicatedEvent(state, re, setup.replication.get) case pe: PublishedEventImpl => onPublishedEvent(state, pe) case JournalResponse(r) => onDeleteEventsJournalResponse(r, state.state) case SnapshotterResponse(r) => onDeleteSnapshotResponse(r, state.state) @@ -267,19 +266,19 @@ private[akka] object Running { def onReplicatedEvent( state: Running.RunningState[S], envelope: ReplicatedEventEnvelope[E], - activeActive: ActiveActive): Behavior[InternalProtocol] = { + replication: ReplicationSetup): Behavior[InternalProtocol] = { setup.log.infoN( "Replica {} received replicated event. Replica seqs nrs: {}. Envelope {}", - setup.activeActive, + setup.replication, state.seenPerReplica, envelope) envelope.ack ! ReplicatedEventAck - if (envelope.event.originReplica != activeActive.replicaId && !alreadySeen(envelope.event)) { + if (envelope.event.originReplica != replication.replicaId && !alreadySeen(envelope.event)) { setup.log.debug( "Saving event [{}] from [{}] as first time", envelope.event.originSequenceNr, envelope.event.originReplica) - handleExternalReplicatedEventPersist(activeActive, envelope.event) + handleExternalReplicatedEventPersist(replication, envelope.event) } else { setup.log.debug( "Filtering event [{}] from [{}] as it was already seen", @@ -290,19 +289,20 @@ private[akka] object Running { } def onPublishedEvent(state: Running.RunningState[S], event: PublishedEventImpl): Behavior[InternalProtocol] = { - val newBehavior: Behavior[InternalProtocol] = setup.activeActive match { + val newBehavior: Behavior[InternalProtocol] = setup.replication match { case None => - setup.log - .warn("Received published event for [{}] but not an active active actor, dropping", event.persistenceId) + setup.log.warn( + "Received published event for [{}] but not an Replicated Event Sourcing actor, dropping", + event.persistenceId) this - case Some(activeActive) => + case Some(replication) => event.replicatedMetaData match { case None => setup.log.warn("Received published event for [{}] but with no replicated metadata, dropping") this case Some(replicatedEventMetaData) => - onPublishedEvent(state, activeActive, replicatedEventMetaData, event) + onPublishedEvent(state, replication, replicatedEventMetaData, event) } } tryUnstashOne(newBehavior) @@ -310,7 +310,7 @@ private[akka] object Running { private def onPublishedEvent( state: Running.RunningState[S], - activeActive: ActiveActive, + replication: ReplicationSetup, replicatedMetadata: ReplicatedPublishedEventMetaData, event: PublishedEventImpl): Behavior[InternalProtocol] = { val log = setup.log @@ -320,18 +320,18 @@ private[akka] object Running { if (!setup.persistenceId.id.startsWith(idPrefix)) { log.warn("Ignoring published replicated event for the wrong actor [{}]", event.persistenceId) this - } else if (originReplicaId == activeActive.replicaId) { + } else if (originReplicaId == replication.replicaId) { if (log.isDebugEnabled) log.debug( "Ignoring published replicated event with seqNr [{}] from our own replica id [{}]", event.sequenceNumber, originReplicaId) this - } else if (!activeActive.allReplicas.contains(originReplicaId)) { + } else if (!replication.allReplicas.contains(originReplicaId)) { log.warnN( - "Received published replicated event from replica [{}], which is unknown. Active active must be set up with a list of all replicas (known are [{}]).", + "Received published replicated event from replica [{}], which is unknown. Replicated Event Sourcing must be set up with a list of all replicas (known are [{}]).", originReplicaId, - activeActive.allReplicas.mkString(", ")) + replication.allReplicas.mkString(", ")) this } else { val expectedSequenceNumber = state.seenPerReplica(originReplicaId) + 1 @@ -369,7 +369,7 @@ private[akka] object Running { state.replicationControl.get(originReplicaId).foreach(_.fastForward(event.sequenceNumber)) handleExternalReplicatedEventPersist( - activeActive, + replication, ReplicatedEvent( event.event.asInstanceOf[E], originReplicaId, @@ -386,15 +386,15 @@ private[akka] object Running { this } - def withContext[A](aa: ActiveActive, withActiveActive: ActiveActive => Unit, f: () => A): A = { - withActiveActive(aa) + def withContext[A](aa: ReplicationSetup, withReplication: ReplicationSetup => Unit, f: () => A): A = { + withReplication(aa) val result = f() aa.clearContext() result } private def handleExternalReplicatedEventPersist( - activeActive: ActiveActive, + replication: ReplicationSetup, event: ReplicatedEvent[E]): Behavior[InternalProtocol] = { _currentSequenceNumber = state.seqNr + 1 val isConcurrent: Boolean = event.originVersion <> state.version @@ -410,7 +410,7 @@ private[akka] object Running { isConcurrent) val newState: RunningState[S] = withContext( - activeActive, + replication, aa => aa.setContext(recoveryRunning = false, event.originReplica, concurrent = isConcurrent), () => state.applyEvent(setup, event.event)) @@ -442,7 +442,7 @@ private[akka] object Running { // also, ensure that there is an event handler for each single event _currentSequenceNumber = state.seqNr + 1 - val newState: RunningState[S] = setup.activeActive match { + val newState: RunningState[S] = setup.replication match { case Some(aa) => // set concurrent to false, local events are never concurrent withContext( @@ -456,7 +456,7 @@ private[akka] object Running { val eventToPersist = adaptEvent(event) val eventAdapterManifest = setup.eventAdapter.manifest(event) - val newState2 = setup.activeActive match { + val newState2 = setup.replication match { case Some(aa) => val updatedVersion = newState.version.updated(aa.replicaId.id, _currentSequenceNumber) val r = internalPersist( @@ -494,7 +494,7 @@ private[akka] object Running { // also, ensure that there is an event handler for each single event _currentSequenceNumber = state.seqNr - val metadataTemplate: Option[ReplicatedEventMetadata] = setup.activeActive match { + val metadataTemplate: Option[ReplicatedEventMetadata] = setup.replication match { case Some(aa) => aa.setContext(recoveryRunning = false, aa.replicaId, concurrent = false) // local events are never concurrent Some(ReplicatedEventMetadata(aa.replicaId, 0L, state.version, concurrent = false)) // we replace it with actual seqnr later @@ -524,7 +524,7 @@ private[akka] object Running { case None => None } - currentState = setup.activeActive match { + currentState = setup.replication match { case Some(aa) => withContext( aa, @@ -679,7 +679,7 @@ private[akka] object Running { onWriteSuccess(setup.context, p) if (setup.publishEvents) { - val meta = setup.activeActive.map(aa => new ReplicatedPublishedEventMetaData(aa.replicaId, state.version)) + val meta = setup.replication.map(aa => new ReplicatedPublishedEventMetaData(aa.replicaId, state.version)) context.system.eventStream ! EventStream.Publish( PublishedEventImpl(setup.persistenceId, p.sequenceNr, p.payload, p.timestamp, meta)) } diff --git a/akka-persistence-typed/src/main/scala/akka/persistence/typed/javadsl/EventSourcedBehavior.scala b/akka-persistence-typed/src/main/scala/akka/persistence/typed/javadsl/EventSourcedBehavior.scala index 00c6008121..fc0a1cf57c 100644 --- a/akka-persistence-typed/src/main/scala/akka/persistence/typed/javadsl/EventSourcedBehavior.scala +++ b/akka-persistence-typed/src/main/scala/akka/persistence/typed/javadsl/EventSourcedBehavior.scala @@ -179,7 +179,7 @@ abstract class EventSourcedBehavior[Command, Event, State] private[akka] ( * INTERNAL API: DeferredBehavior init, not for user extension */ @InternalApi override def apply(context: typed.TypedActorContext[Command]): Behavior[Command] = { - // Note: duplicated in ActiveActiveEventSourcedBehavior to not break source compatibility + // Note: duplicated in ReplicatedEventSourcedBehavior to not break source compatibility val snapshotWhen: (State, Event, Long) => Boolean = (state, event, seqNr) => shouldSnapshot(state, event, seqNr) val tagger: Event => Set[String] = { event => diff --git a/akka-persistence-typed/src/main/scala/akka/persistence/typed/javadsl/ActiveActiveEventSourcedBehavior.scala b/akka-persistence-typed/src/main/scala/akka/persistence/typed/javadsl/ReplicatedEventSourcedBehavior.scala similarity index 79% rename from akka-persistence-typed/src/main/scala/akka/persistence/typed/javadsl/ActiveActiveEventSourcedBehavior.scala rename to akka-persistence-typed/src/main/scala/akka/persistence/typed/javadsl/ReplicatedEventSourcedBehavior.scala index 9e315b04f1..7346615e8f 100644 --- a/akka-persistence-typed/src/main/scala/akka/persistence/typed/javadsl/ActiveActiveEventSourcedBehavior.scala +++ b/akka-persistence-typed/src/main/scala/akka/persistence/typed/javadsl/ReplicatedEventSourcedBehavior.scala @@ -12,23 +12,26 @@ import akka.actor.typed.TypedActorContext import akka.annotation.ApiMayChange import akka.annotation.InternalApi import akka.persistence.typed.internal +import akka.persistence.typed.internal.ReplicationContextImpl import akka.persistence.typed.internal.EffectImpl -import akka.persistence.typed.scaladsl.ActiveActiveContextImpl +/** + * Base class for replicated event sourced behaviors. + */ @ApiMayChange -abstract class ActiveActiveEventSourcedBehavior[Command, Event, State]( - activeActiveContext: ActiveActiveContext, +abstract class ReplicatedEventSourcedBehavior[Command, Event, State]( + replicationContext: ReplicationContext, onPersistFailure: Optional[BackoffSupervisorStrategy]) - extends EventSourcedBehavior[Command, Event, State](activeActiveContext.persistenceId, onPersistFailure) { + extends EventSourcedBehavior[Command, Event, State](replicationContext.persistenceId, onPersistFailure) { - def this(activeActiveContext: ActiveActiveContext) = this(activeActiveContext, Optional.empty()) + def this(replicationContext: ReplicationContext) = this(replicationContext, Optional.empty()) /** * Override and return true to publish events to the system event stream as [[akka.persistence.typed.PublishedEvent]] after they have been persisted */ def withEventPublishing: Boolean = false - protected def getActiveActiveContext(): ActiveActiveContext = activeActiveContext + protected def getReplicationContext(): ReplicationContext = replicationContext /** * INTERNAL API: DeferredBehavior init, not for user extension @@ -59,7 +62,7 @@ abstract class ActiveActiveEventSourcedBehavior[Command, Event, State]( .withSnapshotPluginId(snapshotPluginId) .withRecovery(recovery.asScala) // context not user extendable so there should never be any other impls - .withActiveActive(activeActiveContext.asInstanceOf[ActiveActiveContextImpl]) + .withReplication(replicationContext.asInstanceOf[ReplicationContextImpl]) val handler = signalHandler() val behaviorWithSignalHandler = diff --git a/akka-persistence-typed/src/main/scala/akka/persistence/typed/javadsl/ActiveActiveEventSourcing.scala b/akka-persistence-typed/src/main/scala/akka/persistence/typed/javadsl/ReplicatedEventSourcing.scala similarity index 71% rename from akka-persistence-typed/src/main/scala/akka/persistence/typed/javadsl/ActiveActiveEventSourcing.scala rename to akka-persistence-typed/src/main/scala/akka/persistence/typed/javadsl/ReplicatedEventSourcing.scala index d59b812bc6..f5ed8f29be 100644 --- a/akka-persistence-typed/src/main/scala/akka/persistence/typed/javadsl/ActiveActiveEventSourcing.scala +++ b/akka-persistence-typed/src/main/scala/akka/persistence/typed/javadsl/ReplicatedEventSourcing.scala @@ -11,36 +11,63 @@ import java.util.{ Map => JMap } import akka.annotation.DoNotInherit import akka.persistence.typed.PersistenceId import akka.persistence.typed.ReplicaId -import akka.persistence.typed.scaladsl.ActiveActiveContextImpl +import akka.persistence.typed.internal.ReplicationContextImpl import akka.util.ccompat.JavaConverters._ /** - * Provides access to Active Active specific state + * Provides access to replication specific state * * Not for user extension */ @DoNotInherit -trait ActiveActiveContext { - def origin: ReplicaId - def concurrent: Boolean +trait ReplicationContext { + + /** + * @return The replica id of this replicated event sourced actor + */ def replicaId: ReplicaId + + /** + * @return The ids of all replicas of this replicated event sourced actor + */ def getAllReplicas: JSet[ReplicaId] + + /** + * @return The unique id of this replica, including the replica id + */ def persistenceId: PersistenceId - def recoveryRunning: Boolean + + /** + * @return The unique id of this replica, not including the replica id + */ def entityId: String + + /** + * Must only be called from the event handler + * @return true when the event handler is invoked during recovery. + */ + def recoveryRunning: Boolean + + /** + * Must only be called from the event handler + * @return the replica id where the current event was persisted + */ + def origin: ReplicaId + + /** + * Must only be called from the event handler + * @return true if this event happened concurrent with an event from another replica + */ + def concurrent: Boolean + + /** + * @return a timestamp that will always be increasing (is monotonic) + */ def currentTimeMillis(): Long } -/** - * Factory to create an instance of an ActiveActiveEventSourcedBehavior - */ -@FunctionalInterface -trait ActiveActiveBehaviorFactory[Command, Event, State] { - def apply(aaContext: ActiveActiveContext): ActiveActiveEventSourcedBehavior[Command, Event, State] -} - -object ActiveActiveEventSourcing { +object ReplicatedEventSourcing { /** * Initialize a replicated event sourced behavior where all entity replicas are stored in the same journal. @@ -63,7 +90,7 @@ object ActiveActiveEventSourcing { replicaId: ReplicaId, allReplicaIds: JSet[ReplicaId], queryPluginId: String, - behaviorFactory: JFunction[ActiveActiveContext, EventSourcedBehavior[Command, Event, State]]) + behaviorFactory: JFunction[ReplicationContext, EventSourcedBehavior[Command, Event, State]]) : EventSourcedBehavior[Command, Event, State] = create(entityId, replicaId, allReplicaIds.asScala.map(id => id -> queryPluginId).toMap.asJava, behaviorFactory) @@ -87,9 +114,9 @@ object ActiveActiveEventSourcing { entityId: String, replicaId: ReplicaId, allReplicasAndQueryPlugins: JMap[ReplicaId, String], - eventSourcedBehaviorFactory: JFunction[ActiveActiveContext, EventSourcedBehavior[Command, Event, State]]) + eventSourcedBehaviorFactory: JFunction[ReplicationContext, EventSourcedBehavior[Command, Event, State]]) : EventSourcedBehavior[Command, Event, State] = { - val context = new ActiveActiveContextImpl(entityId, replicaId, allReplicasAndQueryPlugins.asScala.toMap) + val context = new ReplicationContextImpl(entityId, replicaId, allReplicasAndQueryPlugins.asScala.toMap) eventSourcedBehaviorFactory(context) } diff --git a/akka-persistence-typed/src/main/scala/akka/persistence/typed/scaladsl/EventSourcedBehavior.scala b/akka-persistence-typed/src/main/scala/akka/persistence/typed/scaladsl/EventSourcedBehavior.scala index b40d3a327e..0c4c114ed8 100644 --- a/akka-persistence-typed/src/main/scala/akka/persistence/typed/scaladsl/EventSourcedBehavior.scala +++ b/akka-persistence-typed/src/main/scala/akka/persistence/typed/scaladsl/EventSourcedBehavior.scala @@ -4,7 +4,6 @@ package akka.persistence.typed.scaladsl -import scala.annotation.tailrec import akka.actor.typed.BackoffSupervisorStrategy import akka.actor.typed.Behavior import akka.actor.typed.Signal @@ -13,45 +12,18 @@ import akka.actor.typed.internal.InterceptorImpl import akka.actor.typed.internal.LoggerClass import akka.actor.typed.scaladsl.ActorContext import akka.annotation.ApiMayChange -import akka.annotation.{ DoNotInherit, InternalApi } +import akka.annotation.DoNotInherit +import akka.annotation.InternalApi import akka.persistence.typed.EventAdapter import akka.persistence.typed.PersistenceId -import akka.persistence.typed.ReplicaId import akka.persistence.typed.SnapshotAdapter import akka.persistence.typed.SnapshotSelectionCriteria import akka.persistence.typed.internal._ -import akka.util.OptionVal + +import scala.annotation.tailrec object EventSourcedBehavior { - // FIXME move to internal - @InternalApi - private[akka] final case class ActiveActive( - replicaId: ReplicaId, - allReplicasAndQueryPlugins: Map[ReplicaId, String], - aaContext: ActiveActiveContextImpl) { - - val allReplicas: Set[ReplicaId] = allReplicasAndQueryPlugins.keySet - - /** - * Must only be called on the same thread that will execute the user code - */ - def setContext(recoveryRunning: Boolean, originReplica: ReplicaId, concurrent: Boolean): Unit = { - aaContext._currentThread = OptionVal.Some(Thread.currentThread()) - aaContext._recoveryRunning = recoveryRunning - aaContext._concurrent = concurrent - aaContext._origin = originReplica - } - - def clearContext(): Unit = { - aaContext._currentThread = OptionVal.None - aaContext._recoveryRunning = false - aaContext._concurrent = false - aaContext._origin = null - } - - } - /** * Type alias for the command handler function that defines how to act on commands. * @@ -175,8 +147,6 @@ object EventSourcedBehavior { */ def withJournalPluginId(id: String): EventSourcedBehavior[Command, Event, State] - private[akka] def withActiveActive(context: ActiveActiveContextImpl): EventSourcedBehavior[Command, Event, State] - /** * Change the snapshot store plugin id that this actor should use. */ @@ -253,4 +223,10 @@ object EventSourcedBehavior { */ @ApiMayChange def withEventPublishing(): EventSourcedBehavior[Command, Event, State] + + /** + * INTERNAL API + */ + @InternalApi + private[akka] def withReplication(context: ReplicationContextImpl): EventSourcedBehavior[Command, Event, State] } diff --git a/akka-persistence-typed/src/main/scala/akka/persistence/typed/scaladsl/ActiveActiveEventSourcing.scala b/akka-persistence-typed/src/main/scala/akka/persistence/typed/scaladsl/ReplicatedEventSourcing.scala similarity index 54% rename from akka-persistence-typed/src/main/scala/akka/persistence/typed/scaladsl/ActiveActiveEventSourcing.scala rename to akka-persistence-typed/src/main/scala/akka/persistence/typed/scaladsl/ReplicatedEventSourcing.scala index 7570fa3b31..6cf395f51a 100644 --- a/akka-persistence-typed/src/main/scala/akka/persistence/typed/scaladsl/ActiveActiveEventSourcing.scala +++ b/akka-persistence-typed/src/main/scala/akka/persistence/typed/scaladsl/ReplicatedEventSourcing.scala @@ -4,90 +4,65 @@ package akka.persistence.typed.scaladsl +import akka.annotation.DoNotInherit import akka.persistence.typed.PersistenceId import akka.persistence.typed.ReplicaId -import akka.util.{ OptionVal, WallClock } +import akka.persistence.typed.internal.ReplicationContextImpl -import akka.util.ccompat.JavaConverters._ - -// FIXME docs -trait ActiveActiveContext { +/** + * Provides access to replication specific state + * + * Not for user extension + */ +@DoNotInherit +trait ReplicationContext { + /** + * @return The unique id of this replica, including the replica id + */ def persistenceId: PersistenceId + + /** + * @return The replica id of this replicated event sourced actor + */ def replicaId: ReplicaId + + /** + * @return The ids of all replicas of this replicated event sourced actor + */ def allReplicas: Set[ReplicaId] + + /** + * @return The entity id of this replicated event sourced actor (not including the replica id) + */ def entityId: String + /** + * Must only be called from the event handler + * @return the replica id where the current event was persisted + */ def origin: ReplicaId + + /** + * Must only be called from the event handler + * @return true if this event happened concurrent with an event from another replica + */ def concurrent: Boolean + + /** + * Must only be called from the event handler + * @return true when the event handler is invoked during recovery. + */ def recoveryRunning: Boolean + + /** + * @return a timestamp that will always be increasing (is monotonic) + */ def currentTimeMillis(): Long } -// FIXME, parts of this can be set during initialisation -// Other fields will be set before executing the event handler as they change per event -// https://github.com/akka/akka/issues/29258 -private[akka] class ActiveActiveContextImpl( - val entityId: String, - val replicaId: ReplicaId, - val replicasAndQueryPlugins: Map[ReplicaId, String]) - extends ActiveActiveContext - with akka.persistence.typed.javadsl.ActiveActiveContext { - val allReplicas: Set[ReplicaId] = replicasAndQueryPlugins.keySet - - // these are not volatile as they are set on the same thread as they should be accessed - var _origin: ReplicaId = null - var _recoveryRunning: Boolean = false - var _concurrent: Boolean = false - var _currentThread: OptionVal[Thread] = OptionVal.None - - private def checkAccess(functionName: String): Unit = { - val callerThread = Thread.currentThread() - def error() = - throw new UnsupportedOperationException( - s"Unsupported access to ActiveActiveContext operation from the outside of event handler. " + - s"$functionName can only be called from the event handler") - _currentThread match { - case OptionVal.Some(t) => - if (callerThread ne t) error() - case OptionVal.None => - error() - } - } - - /** - * The origin of the current event. - * Undefined result if called from anywhere other than an event handler. - */ - override def origin: ReplicaId = { - checkAccess("origin") - _origin - } - - /** - * Whether the happened concurrently with an event from another replica. - * Undefined result if called from any where other than an event handler. - */ - override def concurrent: Boolean = { - checkAccess("concurrent") - _concurrent - } - - override def persistenceId: PersistenceId = PersistenceId.replicatedUniqueId(entityId, replicaId) - - override def currentTimeMillis(): Long = { - WallClock.AlwaysIncreasingClock.currentTimeMillis() - } - override def recoveryRunning: Boolean = { - checkAccess("recoveryRunning") - _recoveryRunning - } - - override def getAllReplicas: java.util.Set[ReplicaId] = allReplicas.asJava -} - -object ActiveActiveEventSourcing { +object ReplicatedEventSourcing { /** * Initialize a replicated event sourced behavior where all entity replicas are stored in the same journal. @@ -110,7 +85,7 @@ object ActiveActiveEventSourcing { replicaId: ReplicaId, allReplicaIds: Set[ReplicaId], queryPluginId: String)( - eventSourcedBehaviorFactory: ActiveActiveContext => EventSourcedBehavior[Command, Event, State]) + eventSourcedBehaviorFactory: ReplicationContext => EventSourcedBehavior[Command, Event, State]) : EventSourcedBehavior[Command, Event, State] = apply(entityId, replicaId, allReplicaIds.map(id => id -> queryPluginId).toMap)(eventSourcedBehaviorFactory) @@ -134,10 +109,10 @@ object ActiveActiveEventSourcing { entityId: String, replicaId: ReplicaId, allReplicasAndQueryPlugins: Map[ReplicaId, String])( - eventSourcedBehaviorFactory: ActiveActiveContext => EventSourcedBehavior[Command, Event, State]) + eventSourcedBehaviorFactory: ReplicationContext => EventSourcedBehavior[Command, Event, State]) : EventSourcedBehavior[Command, Event, State] = { - val context = new ActiveActiveContextImpl(entityId, replicaId, allReplicasAndQueryPlugins) - eventSourcedBehaviorFactory(context).withActiveActive(context) + val context = new ReplicationContextImpl(entityId, replicaId, allReplicasAndQueryPlugins) + eventSourcedBehaviorFactory(context).withReplication(context) } } diff --git a/akka-persistence-typed/src/main/scala/akka/persistence/typed/serialization/ActiveActiveSerializer.scala b/akka-persistence-typed/src/main/scala/akka/persistence/typed/serialization/ReplicatedEventSourcingSerializer.scala similarity index 75% rename from akka-persistence-typed/src/main/scala/akka/persistence/typed/serialization/ActiveActiveSerializer.scala rename to akka-persistence-typed/src/main/scala/akka/persistence/typed/serialization/ReplicatedEventSourcingSerializer.scala index 51633d6eff..5c4b4dc25d 100644 --- a/akka-persistence-typed/src/main/scala/akka/persistence/typed/serialization/ActiveActiveSerializer.scala +++ b/akka-persistence-typed/src/main/scala/akka/persistence/typed/serialization/ReplicatedEventSourcingSerializer.scala @@ -22,9 +22,13 @@ import akka.serialization.{ BaseSerializer, SerializerWithStringManifest } import scala.annotation.tailrec import akka.util.ccompat.JavaConverters._ + import scala.collection.immutable.TreeMap -object ActiveActiveSerializer { +/** + * INTERNAL API + */ +@InternalApi private[akka] object ReplicatedEventSourcingSerializer { object Comparator extends Comparator[Payload] { override def compare(a: Payload, b: Payload): Int = { val aByteString = a.getEnclosedMessage @@ -53,7 +57,7 @@ object ActiveActiveSerializer { /** * INTERNAL API */ -@InternalApi private[akka] final class ActiveActiveSerializer(val system: ExtendedActorSystem) +@InternalApi private[akka] final class ReplicatedEventSourcingSerializer(val system: ExtendedActorSystem) extends SerializerWithStringManifest with BaseSerializer { @@ -132,23 +136,34 @@ object ActiveActiveSerializer { } def counterFromBinary(bytes: Array[Byte]): Counter = - Counter(BigInt(ActiveActive.Counter.parseFrom(bytes).getValue.toByteArray)) + Counter(BigInt(ReplicatedEventSourcing.Counter.parseFrom(bytes).getValue.toByteArray)) def counterUpdatedFromBinary(bytes: Array[Byte]): Counter.Updated = - Counter.Updated(BigInt(ActiveActive.CounterUpdate.parseFrom(bytes).getDelta.toByteArray)) + Counter.Updated(BigInt(ReplicatedEventSourcing.CounterUpdate.parseFrom(bytes).getDelta.toByteArray)) def counterToProtoByteArray(counter: Counter): Array[Byte] = - ActiveActive.Counter.newBuilder().setValue(ByteString.copyFrom(counter.value.toByteArray)).build().toByteArray + ReplicatedEventSourcing.Counter + .newBuilder() + .setValue(ByteString.copyFrom(counter.value.toByteArray)) + .build() + .toByteArray def counterUpdatedToProtoBufByteArray(updated: Counter.Updated): Array[Byte] = - ActiveActive.CounterUpdate.newBuilder().setDelta(ByteString.copyFrom(updated.delta.toByteArray)).build().toByteArray + ReplicatedEventSourcing.CounterUpdate + .newBuilder() + .setDelta(ByteString.copyFrom(updated.delta.toByteArray)) + .build() + .toByteArray - def orsetToProto(orset: ORSet[_]): ActiveActive.ORSet = + def orsetToProto(orset: ORSet[_]): ReplicatedEventSourcing.ORSet = orsetToProtoImpl(orset.asInstanceOf[ORSet[Any]]) - private def orsetToProtoImpl(orset: ORSet[Any]): ActiveActive.ORSet = { + private def orsetToProtoImpl(orset: ORSet[Any]): ReplicatedEventSourcing.ORSet = { val b = - ActiveActive.ORSet.newBuilder().setOriginDc(orset.originReplica).setVvector(versionVectorToProto(orset.vvector)) + ReplicatedEventSourcing.ORSet + .newBuilder() + .setOriginDc(orset.originReplica) + .setVvector(versionVectorToProto(orset.vvector)) // using java collections and sorting for performance (avoid conversions) val stringElements = new ArrayList[String] val intElements = new ArrayList[Integer] @@ -194,7 +209,7 @@ object ActiveActiveSerializer { addDots(longElements) } if (!otherElements.isEmpty) { - Collections.sort(otherElements, ActiveActiveSerializer.Comparator) + Collections.sort(otherElements, ReplicatedEventSourcingSerializer.Comparator) b.addAllOtherElements(otherElements) addDots(otherElements) } @@ -203,7 +218,7 @@ object ActiveActiveSerializer { } def replicatedEventMetadataToProtoByteArray(rem: ReplicatedEventMetadata): Array[Byte] = { - ActiveActive.ReplicatedEventMetadata + ReplicatedEventSourcing.ReplicatedEventMetadata .newBuilder() .setOriginSequenceNr(rem.originSequenceNr) .setConcurrent(rem.concurrent) @@ -214,7 +229,7 @@ object ActiveActiveSerializer { } def replicatedSnapshotMetadataToByteArray(rsm: ReplicatedSnapshotMetadata): Array[Byte] = { - ActiveActive.ReplicatedSnapshotMetadata + ReplicatedEventSourcing.ReplicatedSnapshotMetadata .newBuilder() .setVersion(versionVectorToProto(rsm.version)) .addAllSeenPerReplica(rsm.seenPerReplica.map(seenToProto).asJava) @@ -222,35 +237,39 @@ object ActiveActiveSerializer { .toByteArray } - def seenToProto(t: (ReplicaId, Long)): ActiveActive.ReplicatedSnapshotMetadata.Seen = { - ActiveActive.ReplicatedSnapshotMetadata.Seen.newBuilder().setReplicaId(t._1.id).setSequenceNr(t._2).build() + def seenToProto(t: (ReplicaId, Long)): ReplicatedEventSourcing.ReplicatedSnapshotMetadata.Seen = { + ReplicatedEventSourcing.ReplicatedSnapshotMetadata.Seen + .newBuilder() + .setReplicaId(t._1.id) + .setSequenceNr(t._2) + .build() } def orsetFromBinary(bytes: Array[Byte]): ORSet[Any] = - orsetFromProto(ActiveActive.ORSet.parseFrom(bytes)) + orsetFromProto(ReplicatedEventSourcing.ORSet.parseFrom(bytes)) private def orsetAddFromBinary(bytes: Array[Byte]): ORSet.AddDeltaOp[Any] = - new ORSet.AddDeltaOp(orsetFromProto(ActiveActive.ORSet.parseFrom(bytes))) + new ORSet.AddDeltaOp(orsetFromProto(ReplicatedEventSourcing.ORSet.parseFrom(bytes))) private def orsetRemoveFromBinary(bytes: Array[Byte]): ORSet.RemoveDeltaOp[Any] = - new ORSet.RemoveDeltaOp(orsetFromProto(ActiveActive.ORSet.parseFrom(bytes))) + new ORSet.RemoveDeltaOp(orsetFromProto(ReplicatedEventSourcing.ORSet.parseFrom(bytes))) private def orsetFullFromBinary(bytes: Array[Byte]): ORSet.FullStateDeltaOp[Any] = - new ORSet.FullStateDeltaOp(orsetFromProto(ActiveActive.ORSet.parseFrom(bytes))) + new ORSet.FullStateDeltaOp(orsetFromProto(ReplicatedEventSourcing.ORSet.parseFrom(bytes))) - private def orsetDeltaGroupToProto(deltaGroup: ORSet.DeltaGroup[_]): ActiveActive.ORSetDeltaGroup = { - def createEntry(opType: ActiveActive.ORSetDeltaOp, u: ORSet[_]) = { - ActiveActive.ORSetDeltaGroup.Entry.newBuilder().setOperation(opType).setUnderlying(orsetToProto(u)) + private def orsetDeltaGroupToProto(deltaGroup: ORSet.DeltaGroup[_]): ReplicatedEventSourcing.ORSetDeltaGroup = { + def createEntry(opType: ReplicatedEventSourcing.ORSetDeltaOp, u: ORSet[_]) = { + ReplicatedEventSourcing.ORSetDeltaGroup.Entry.newBuilder().setOperation(opType).setUnderlying(orsetToProto(u)) } - val b = ActiveActive.ORSetDeltaGroup.newBuilder() + val b = ReplicatedEventSourcing.ORSetDeltaGroup.newBuilder() deltaGroup.ops.foreach { case ORSet.AddDeltaOp(u) => - b.addEntries(createEntry(ActiveActive.ORSetDeltaOp.Add, u)) + b.addEntries(createEntry(ReplicatedEventSourcing.ORSetDeltaOp.Add, u)) case ORSet.RemoveDeltaOp(u) => - b.addEntries(createEntry(ActiveActive.ORSetDeltaOp.Remove, u)) + b.addEntries(createEntry(ReplicatedEventSourcing.ORSetDeltaOp.Remove, u)) case ORSet.FullStateDeltaOp(u) => - b.addEntries(createEntry(ActiveActive.ORSetDeltaOp.Full, u)) + b.addEntries(createEntry(ReplicatedEventSourcing.ORSetDeltaOp.Full, u)) case ORSet.DeltaGroup(_) => throw new IllegalArgumentException("ORSet.DeltaGroup should not be nested") } @@ -258,14 +277,14 @@ object ActiveActiveSerializer { } private def orsetDeltaGroupFromBinary(bytes: Array[Byte]): ORSet.DeltaGroup[Any] = { - val deltaGroup = ActiveActive.ORSetDeltaGroup.parseFrom(bytes) + val deltaGroup = ReplicatedEventSourcing.ORSetDeltaGroup.parseFrom(bytes) val ops: Vector[ORSet.DeltaOp] = deltaGroup.getEntriesList.asScala.map { entry => - if (entry.getOperation == ActiveActive.ORSetDeltaOp.Add) + if (entry.getOperation == ReplicatedEventSourcing.ORSetDeltaOp.Add) ORSet.AddDeltaOp(orsetFromProto(entry.getUnderlying)) - else if (entry.getOperation == ActiveActive.ORSetDeltaOp.Remove) + else if (entry.getOperation == ReplicatedEventSourcing.ORSetDeltaOp.Remove) ORSet.RemoveDeltaOp(orsetFromProto(entry.getUnderlying)) - else if (entry.getOperation == ActiveActive.ORSetDeltaOp.Full) + else if (entry.getOperation == ReplicatedEventSourcing.ORSetDeltaOp.Full) ORSet.FullStateDeltaOp(orsetFromProto(entry.getUnderlying)) else throw new NotSerializableException(s"Unknow ORSet delta operation ${entry.getOperation}") @@ -273,7 +292,7 @@ object ActiveActiveSerializer { ORSet.DeltaGroup(ops) } - def orsetFromProto(orset: ActiveActive.ORSet): ORSet[Any] = { + def orsetFromProto(orset: ReplicatedEventSourcing.ORSet): ORSet[Any] = { val elements: Iterator[Any] = (orset.getStringElementsList.iterator.asScala ++ orset.getIntElementsList.iterator.asScala ++ @@ -286,18 +305,19 @@ object ActiveActiveSerializer { new ORSet(orset.getOriginDc, elementsMap, vvector = versionVectorFromProto(orset.getVvector)) } - def versionVectorToProto(versionVector: VersionVector): ActiveActive.VersionVector = { - val b = ActiveActive.VersionVector.newBuilder() + def versionVectorToProto(versionVector: VersionVector): ReplicatedEventSourcing.VersionVector = { + val b = ReplicatedEventSourcing.VersionVector.newBuilder() versionVector.versionsIterator.foreach { - case (key, value) => b.addEntries(ActiveActive.VersionVector.Entry.newBuilder().setKey(key).setVersion(value)) + case (key, value) => + b.addEntries(ReplicatedEventSourcing.VersionVector.Entry.newBuilder().setKey(key).setVersion(value)) } b.build() } def versionVectorFromBinary(bytes: Array[Byte]): VersionVector = - versionVectorFromProto(ActiveActive.VersionVector.parseFrom(bytes)) + versionVectorFromProto(ReplicatedEventSourcing.VersionVector.parseFrom(bytes)) - def versionVectorFromProto(versionVector: ActiveActive.VersionVector): VersionVector = { + def versionVectorFromProto(versionVector: ReplicatedEventSourcing.VersionVector): VersionVector = { val entries = versionVector.getEntriesList if (entries.isEmpty) VersionVector.empty @@ -311,7 +331,7 @@ object ActiveActiveSerializer { } def replicatedEventMetadataFromBinary(bytes: Array[Byte]): ReplicatedEventMetadata = { - val parsed = ActiveActive.ReplicatedEventMetadata.parseFrom(bytes) + val parsed = ReplicatedEventSourcing.ReplicatedEventMetadata.parseFrom(bytes) ReplicatedEventMetadata( ReplicaId(parsed.getOriginReplica), parsed.getOriginSequenceNr, @@ -320,7 +340,7 @@ object ActiveActiveSerializer { } def replicatedSnapshotMetadataFromBinary(bytes: Array[Byte]): ReplicatedSnapshotMetadata = { - val parsed = ActiveActive.ReplicatedSnapshotMetadata.parseFrom(bytes) + val parsed = ReplicatedEventSourcing.ReplicatedSnapshotMetadata.parseFrom(bytes) ReplicatedSnapshotMetadata( versionVectorFromProto(parsed.getVersion), parsed.getSeenPerReplicaList.asScala.map(seen => ReplicaId(seen.getReplicaId) -> seen.getSequenceNr).toMap) diff --git a/akka-persistence-typed/src/test/scala/akka/persistence/typed/ActiveActiveSerializationSpec.scala b/akka-persistence-typed/src/test/scala/akka/persistence/typed/ReplicatedEventSourcingSerializationSpec.scala similarity index 91% rename from akka-persistence-typed/src/test/scala/akka/persistence/typed/ActiveActiveSerializationSpec.scala rename to akka-persistence-typed/src/test/scala/akka/persistence/typed/ReplicatedEventSourcingSerializationSpec.scala index 84b4fb2991..05c6e3fa43 100644 --- a/akka-persistence-typed/src/test/scala/akka/persistence/typed/ActiveActiveSerializationSpec.scala +++ b/akka-persistence-typed/src/test/scala/akka/persistence/typed/ReplicatedEventSourcingSerializationSpec.scala @@ -13,12 +13,12 @@ import akka.persistence.typed.internal.ReplicatedSnapshotMetadata import akka.persistence.typed.internal.VersionVector import org.scalatest.wordspec.AnyWordSpecLike -class ActiveActiveSerializationSpec +class ReplicatedEventSourcingSerializationSpec extends ScalaTestWithActorTestKit(ClusterSingletonPersistenceSpec.config) with AnyWordSpecLike with LogCapturing { - "The ActiveActive components that needs to be serializable" must { + "The Replicated Event Sourcing components that needs to be serializable" must { "be serializable" in { serializationTestKit.verifySerialization( diff --git a/akka-persistence-typed/src/test/scala/akka/persistence/typed/scaladsl/EventSourcedBehaviorWatchSpec.scala b/akka-persistence-typed/src/test/scala/akka/persistence/typed/scaladsl/EventSourcedBehaviorWatchSpec.scala index b3175e5777..ef52d0ab7e 100644 --- a/akka-persistence-typed/src/test/scala/akka/persistence/typed/scaladsl/EventSourcedBehaviorWatchSpec.scala +++ b/akka-persistence-typed/src/test/scala/akka/persistence/typed/scaladsl/EventSourcedBehaviorWatchSpec.scala @@ -67,7 +67,7 @@ class EventSourcedBehaviorWatchSpec holdingRecoveryPermit = false, settings = settings, stashState = new StashState(context.asInstanceOf[ActorContext[InternalProtocol]], settings), - activeActive = None, + replication = None, publishEvents = false) "A typed persistent parent actor watching a child" must { diff --git a/akka-persistence/src/main/scala/akka/persistence/SnapshotProtocol.scala b/akka-persistence/src/main/scala/akka/persistence/SnapshotProtocol.scala index 75b9c38480..d1d8f151ef 100644 --- a/akka-persistence/src/main/scala/akka/persistence/SnapshotProtocol.scala +++ b/akka-persistence/src/main/scala/akka/persistence/SnapshotProtocol.scala @@ -11,7 +11,7 @@ import scala.runtime.AbstractFunction3 * @param persistenceId id of persistent actor from which the snapshot was taken. * @param sequenceNr sequence number at which the snapshot was taken. * @param timestamp time at which the snapshot was saved, defaults to 0 when unknown. - * @param metadata a journal can optionally support persisting metadata separate to the domain state, used for active active support + * @param metadata a journal can optionally support persisting metadata separate to the domain state, used for Replicated Event Sourcing support */ @SerialVersionUID(1L) final class SnapshotMetadata(