Class Replicator
- java.lang.Object
-
- akka.cluster.ddata.Replicator
-
- All Implemented Interfaces:
Actor
,ActorLogging
public final class Replicator extends java.lang.Object implements Actor, ActorLogging
A replicated in-memory data store supporting low latency and high availability requirements.The
Replicator
actor takes care of direct replication and gossip based dissemination of Conflict Free Replicated Data Types (CRDTs) to replicas in the the cluster. The data types must be convergent CRDTs and implementReplicatedData
, i.e. they provide a monotonic merge function and the state changes always converge.You can use your own custom
ReplicatedData
orDeltaReplicatedData
types, and several types are provided by this package, such as:- Counters:
GCounter
,PNCounter
- Registers:
LWWRegister
,Flag
- Sets:
GSet
,ORSet
- Maps:
ORMap
,ORMultiMap
,LWWMap
,PNCounterMap
For good introduction to the CRDT subject watch the Eventually Consistent Data Structures talk by Sean Cribbs and and the talk by Mark Shapiro and read the excellent paper A comprehensive study of Convergent and Commutative Replicated Data Types by Mark Shapiro et. al.
The
Replicator
actor must be started on each node in the cluster, or group of nodes tagged with a specific role. It communicates with otherReplicator
instances with the same path (without address) that are running on other nodes . For convenience it can be used with theDistributedData
extension but it can also be started as an ordinary actor using theReplicator.props
. If it is started as an ordinary actor it is important that it is given the same name, started on same path, on all nodes.Delta State Replicated Data Types are supported. delta-CRDT is a way to reduce the need for sending the full state for updates. For example adding element 'c' and 'd' to set {'a', 'b'} would result in sending the delta {'c', 'd'} and merge that with the state on the receiving side, resulting in set {'a', 'b', 'c', 'd'}.
The protocol for replicating the deltas supports causal consistency if the data type is marked with
RequiresCausalDeliveryOfDeltas
. Otherwise it is only eventually consistent. Without causal consistency it means that if elements 'c' and 'd' are added in two separateUpdate
operations these deltas may occasionally be propagated to nodes in different order than the causal order of the updates. For this example it can result in that set {'a', 'b', 'd'} can be seen before element 'c' is seen. Eventually it will be {'a', 'b', 'c', 'd'}.== Update ==
To modify and replicate a
ReplicatedData
value you send aReplicator.Update
message to the localReplicator
. The current data value for thekey
of theUpdate
is passed as parameter to themodify
function of theUpdate
. The function is supposed to return the new value of the data, which will then be replicated according to the given consistency level.The
modify
function is called by theReplicator
actor and must therefore be a pure function that only uses the data parameter and stable fields from enclosing scope. It must for example not accesssender()
reference of an enclosing actor.Update
is intended to only be sent from an actor running in same localActorSystem
as theReplicator
, because themodify
function is typically not serializable.You supply a write consistency level which has the following meaning:
WriteLocal
the value will immediately only be written to the local replica, and later disseminated with gossipWriteTo(n)
the value will immediately be written to at leastn
replicas, including the local replicaWriteMajority
the value will immediately be written to a majority of replicas, i.e. at leastN/2 + 1
replicas, where N is the number of nodes in the cluster (or cluster role group)WriteAll
the value will immediately be written to all nodes in the cluster (or all nodes in the cluster role group)
As reply of the
Update
aReplicator.UpdateSuccess
is sent to the sender of theUpdate
if the value was successfully replicated according to the supplied consistency level within the supplied timeout. Otherwise aReplicator.UpdateFailure
subclass is sent back. Note that aReplicator.UpdateTimeout
reply does not mean that the update completely failed or was rolled back. It may still have been replicated to some nodes, and will eventually be replicated to all nodes with the gossip protocol.You will always see your own writes. For example if you send two
Update
messages changing the value of the samekey
, themodify
function of the second message will see the change that was performed by the firstUpdate
message.In the
Update
message you can pass an optional request context, which theReplicator
does not care about, but is included in the reply messages. This is a convenient way to pass contextual information (e.g. original sender) without having to useask
or local correlation data structures.== Get ==
To retrieve the current value of a data you send
Replicator.Get
message to theReplicator
. You supply a consistency level which has the following meaning:ReadLocal
the value will only be read from the local replicaReadFrom(n)
the value will be read and merged fromn
replicas, including the local replicaReadMajority
the value will be read and merged from a majority of replicas, i.e. at leastN/2 + 1
replicas, where N is the number of nodes in the cluster (or cluster role group)ReadAll
the value will be read and merged from all nodes in the cluster (or all nodes in the cluster role group)
As reply of the
Get
aReplicator.GetSuccess
is sent to the sender of theGet
if the value was successfully retrieved according to the supplied consistency level within the supplied timeout. Otherwise aReplicator.GetFailure
is sent. If the key does not exist the reply will beReplicator.NotFound
.You will always read your own writes. For example if you send a
Update
message followed by aGet
of the samekey
theGet
will retrieve the change that was performed by the precedingUpdate
message. However, the order of the reply messages are not defined, i.e. in the previous example you may receive theGetSuccess
before theUpdateSuccess
.In the
Get
message you can pass an optional request context in the same way as for theUpdate
message, described above. For example the original sender can be passed and replied to after receiving and transformingGetSuccess
.== Subscribe ==
You may also register interest in change notifications by sending
Replicator.Subscribe
message to theReplicator
. It will sendReplicator.Changed
messages to the registered subscriber when the data for the subscribed key is updated. Subscribers will be notified periodically with the configurednotify-subscribers-interval
, and it is also possible to send an explicitReplicator.FlushChanges
message to theReplicator
to notify the subscribers immediately.The subscriber is automatically removed if the subscriber is terminated. A subscriber can also be deregistered with the
Replicator.Unsubscribe
message.== Delete ==
A data entry can be deleted by sending a
Replicator.Delete
message to the local localReplicator
. As reply of theDelete
aReplicator.DeleteSuccess
is sent to the sender of theDelete
if the value was successfully deleted according to the supplied consistency level within the supplied timeout. Otherwise aReplicator.ReplicationDeleteFailure
is sent. Note thatReplicationDeleteFailure
does not mean that the delete completely failed or was rolled back. It may still have been replicated to some nodes, and may eventually be replicated to all nodes.A deleted key cannot be reused again, but it is still recommended to delete unused data entries because that reduces the replication overhead when new nodes join the cluster. Subsequent
Delete
,Update
andGet
requests will be replied withReplicator.DataDeleted
,Replicator.UpdateDataDeleted
andReplicator.GetDataDeleted
respectively. Subscribers will receiveReplicator.Deleted
.In the
Delete
message you can pass an optional request context in the same way as for theUpdate
message, described above. For example the original sender can be passed and replied to after receiving and transformingDeleteSuccess
.== CRDT Garbage ==
One thing that can be problematic with CRDTs is that some data types accumulate history (garbage). For example a
GCounter
keeps track of one counter per node. If aGCounter
has been updated from one node it will associate the identifier of that node forever. That can become a problem for long running systems with many cluster nodes being added and removed. To solve this problem theReplicator
performs pruning of data associated with nodes that have been removed from the cluster. Data types that need pruning have to implementRemovedNodePruning
. The pruning consists of several steps:- When a node is removed from the cluster it is first important that all updates that were
done by that node are disseminated to all other nodes. The pruning will not start before the
maxPruningDissemination
duration has elapsed. The time measurement is stopped when any replica is unreachable, but it's still recommended to configure this with certain margin. It should be in the magnitude of minutes. - The nodes are ordered by their address and the node ordered first is called leader.
The leader initiates the pruning by adding a
PruningInitialized
marker in the data envelope. This is gossiped to all other nodes and they mark it as seen when they receive it. - When the leader sees that all other nodes have seen the
PruningInitialized
marker the leader performs the pruning and changes the marker toPruningPerformed
so that nobody else will redo the pruning. The data envelope with this pruning state is a CRDT itself. The pruning is typically performed by "moving" the part of the data associated with the removed node to the leader node. For example, aGCounter
is aMap
with the node as key and the counts done by that node as value. When pruning the value of the removed node is moved to the entry owned by the leader node. SeeRemovedNodePruning.prune(akka.cluster.UniqueAddress, akka.cluster.UniqueAddress)
. - Thereafter the data is always cleared from parts associated with the removed node so that
it does not come back when merging. See
RemovedNodePruning.pruningCleanup(akka.cluster.UniqueAddress)
- After another
maxPruningDissemination
duration after pruning the last entry from the removed node thePruningPerformed
markers in the data envelope are collapsed into a single tombstone entry, for efficiency. Clients may continue to use old data and therefore all data are always cleared from parts associated with tombstoned nodes.
-
-
Nested Class Summary
Nested Classes Modifier and Type Class Description static class
Replicator.Changed<A extends ReplicatedData>
The data value is retrieved withReplicator.Changed.get(akka.cluster.ddata.Key<T>)
using the typed key.static class
Replicator.Changed$
static interface
Replicator.Command<A extends ReplicatedData>
static class
Replicator.DataDeleted<A extends ReplicatedData>
static class
Replicator.DataDeleted$
static class
Replicator.Delete<A extends ReplicatedData>
Send this message to the localReplicator
to delete a data value for the givenkey
.static class
Replicator.Delete$
static class
Replicator.Deleted<A extends ReplicatedData>
static class
Replicator.Deleted$
static interface
Replicator.DeleteResponse<A extends ReplicatedData>
static class
Replicator.DeleteSuccess<A extends ReplicatedData>
static class
Replicator.DeleteSuccess$
static class
Replicator.Expired<A extends ReplicatedData>
static class
Replicator.Expired$
static class
Replicator.FlushChanges$
Notify subscribers of changes now, otherwise they will be notified periodically with the configurednotify-subscribers-interval
.static class
Replicator.Get<A extends ReplicatedData>
Send this message to the localReplicator
to retrieve a data value for the givenkey
.static class
Replicator.Get$
static class
Replicator.GetDataDeleted<A extends ReplicatedData>
TheReplicator.Get
request couldn't be performed because the entry has been deleted.static class
Replicator.GetDataDeleted$
static class
Replicator.GetFailure<A extends ReplicatedData>
TheReplicator.Get
request could not be fulfill according to the givenconsistency level
andtimeout
.static class
Replicator.GetFailure$
static class
Replicator.GetKeyIds$
INTERNAL APIstatic class
Replicator.GetKeyIdsResult$
static class
Replicator.GetReplicaCount$
Get current number of replicas, including the local replica.static class
Replicator.GetResponse<A extends ReplicatedData>
static class
Replicator.GetSuccess<A extends ReplicatedData>
Reply fromGet
.static class
Replicator.GetSuccess$
static class
Replicator.Internal$
INTERNAL APIstatic class
Replicator.ModifyFailure<A extends ReplicatedData>
If themodify
function of theReplicator.Update
throws an exception the reply message will be thisModifyFailure
message.static class
Replicator.ModifyFailure$
static class
Replicator.NotFound<A extends ReplicatedData>
static class
Replicator.NotFound$
static class
Replicator.ReadAll
static class
Replicator.ReadAll$
static interface
Replicator.ReadConsistency
static class
Replicator.ReadFrom
static class
Replicator.ReadFrom$
static class
Replicator.ReadLocal$
static class
Replicator.ReadMajority
static class
Replicator.ReadMajority$
static class
Replicator.ReadMajorityPlus
ReadMajority
but with the given number ofadditional
nodes added to the majority count.static class
Replicator.ReadMajorityPlus$
static class
Replicator.ReplicaCount
Current number of replicas.static class
Replicator.ReplicaCount$
static class
Replicator.ReplicationDeleteFailure<A extends ReplicatedData>
static class
Replicator.ReplicationDeleteFailure$
static interface
Replicator.ReplicatorMessage
Marker trait for remote messages serialized byReplicatorMessageSerializer
.static class
Replicator.StoreFailure<A extends ReplicatedData>
The local store or direct replication of theReplicator.Update
could not be fulfill according to the givenconsistency level
due to durable store errors.static class
Replicator.StoreFailure$
static class
Replicator.Subscribe<A extends ReplicatedData>
Register a subscriber that will be notified with aReplicator.Changed
message when the value of the givenkey
is changed.static class
Replicator.Subscribe$
static interface
Replicator.SubscribeResponse<A extends ReplicatedData>
static class
Replicator.Unsubscribe<A extends ReplicatedData>
Unregister a subscriber.static class
Replicator.Unsubscribe$
static class
Replicator.Update<A extends ReplicatedData>
static class
Replicator.Update$
static class
Replicator.UpdateDataDeleted<A extends ReplicatedData>
TheReplicator.Update
couldn't be performed because the entry has been deleted.static class
Replicator.UpdateDataDeleted$
static class
Replicator.UpdateFailure<A extends ReplicatedData>
static class
Replicator.UpdateResponse<A extends ReplicatedData>
static class
Replicator.UpdateSuccess<A extends ReplicatedData>
static class
Replicator.UpdateSuccess$
static class
Replicator.UpdateTimeout<A extends ReplicatedData>
The direct replication of theReplicator.Update
could not be fulfill according to the givenconsistency level
andtimeout
.static class
Replicator.UpdateTimeout$
static class
Replicator.WriteAll
static class
Replicator.WriteAll$
static interface
Replicator.WriteConsistency
static class
Replicator.WriteLocal$
static class
Replicator.WriteMajority
static class
Replicator.WriteMajority$
static class
Replicator.WriteMajorityPlus
WriteMajority
but with the given number ofadditional
nodes added to the majority count.static class
Replicator.WriteMajorityPlus$
static class
Replicator.WriteTo
static class
Replicator.WriteTo$
-
Nested classes/interfaces inherited from interface akka.actor.Actor
Actor.emptyBehavior$, Actor.ignoringBehavior$
-
-
Constructor Summary
Constructors Constructor Description Replicator(ReplicatorSettings settings)
-
Method Summary
All Methods Static Methods Instance Methods Concrete Methods Modifier and Type Method Description protected void
akka$actor$Actor$_setter_$context_$eq(ActorContext x$1)
Scala API: Stores the context for this actor, including self, and sender.protected void
akka$actor$Actor$_setter_$self_$eq(ActorRef x$1)
The 'self' field holds the ActorRef for this actor.long
allReachableClockTime()
void
allReachableClockTime_$eq(long x$1)
protected void
aroundReceive(scala.PartialFunction<java.lang.Object,scala.runtime.BoxedUnit> rcv, java.lang.Object msg)
INTERNAL API.scala.collection.immutable.Set<java.lang.String>
changed()
void
changed_$eq(scala.collection.immutable.Set<java.lang.String> x$1)
Cancellable
clockTask()
Cluster
cluster()
void
collectRemovedNodes()
ActorContext
context()
Scala API: Stores the context for this actor, including self, and sender.scala.collection.immutable.Map<java.lang.String,scala.Tuple3<akka.cluster.ddata.Replicator.Internal.DataEnvelope,ByteString,java.lang.Object>>
dataEntries()
void
dataEntries_$eq(scala.collection.immutable.Map<java.lang.String,scala.Tuple3<akka.cluster.ddata.Replicator.Internal.DataEnvelope,ByteString,java.lang.Object>> x$1)
static int
DefaultMajorityMinCap()
void
deleteObsoletePruningPerformed()
java.lang.Object
deltaPropagationSelector()
scala.Option<Cancellable>
deltaPropagationTask()
scala.Tuple2<ByteString,java.lang.Object>
digest(akka.cluster.ddata.Replicator.Internal.DataEnvelope envelope)
scala.collection.immutable.Set<java.lang.String>
durable()
ActorRef
durableStore()
scala.collection.immutable.Set<java.lang.String>
durableWildcards()
scala.collection.immutable.SortedSet<UniqueAddress>
exitingNodes()
void
exitingNodes_$eq(scala.collection.immutable.SortedSet<UniqueAddress> x$1)
boolean
expiryEnabled()
scala.collection.immutable.Map<java.lang.String,scala.concurrent.duration.FiniteDuration>
expiryWildcards()
static Replicator.FlushChanges$
flushChanges()
Java API: TheFlushChanges
instanceboolean
fullStateGossipEnabled()
void
fullStateGossipEnabled_$eq(boolean x$1)
scala.Option<akka.cluster.ddata.Replicator.Internal.DataEnvelope>
getData(java.lang.String key)
long
getDeltaSeqNr(java.lang.String key, UniqueAddress fromNode)
ByteString
getDigest(java.lang.String key)
scala.concurrent.duration.FiniteDuration
getExpiryDuration(java.lang.String key)
static Replicator.GetReplicaCount$
getReplicaCount()
Java API: TheGetReplicaCount
instancelong
getUsedTimestamp(java.lang.String key)
Cancellable
gossipTask()
void
gossipTo(UniqueAddress address)
boolean
hasDurableKeys()
boolean
hasSubscriber(ActorRef subscriber)
void
initRemovedNodePruning()
boolean
isDurable(java.lang.String key)
boolean
isExpired(java.lang.String key)
boolean
isExpired(java.lang.String key, long timestamp)
boolean
isLeader()
boolean
isLocalGet(Replicator.ReadConsistency readConsistency)
boolean
isLocalSender()
boolean
isLocalUpdate(Replicator.WriteConsistency writeConsistency)
boolean
isNodeRemoved(UniqueAddress node, scala.collection.Iterable<java.lang.String> keys)
scala.collection.immutable.SortedSet<UniqueAddress>
joiningNodes()
void
joiningNodes_$eq(scala.collection.immutable.SortedSet<UniqueAddress> x$1)
scala.collection.immutable.TreeSet<Member>
leader()
void
leader_$eq(scala.collection.immutable.TreeSet<Member> x$1)
scala.PartialFunction<java.lang.Object,scala.runtime.BoxedUnit>
load()
boolean
matchingRole(Member m)
long
maxPruningDisseminationNanos()
scala.collection.immutable.SortedSet<Member>
membersByAge()
void
membersByAge_$eq(scala.collection.immutable.SortedSet<Member> x$1)
scala.collection.mutable.HashMap<java.lang.String,scala.collection.mutable.Set<ActorRef>>
newSubscribers()
scala.collection.immutable.SortedSet<UniqueAddress>
nodes()
void
nodes_$eq(scala.collection.immutable.SortedSet<UniqueAddress> x$1)
scala.PartialFunction<java.lang.Object,scala.runtime.BoxedUnit>
normalReceive()
Cancellable
notifyTask()
void
performRemovedNodePruning()
void
postStop()
User overridable callback.void
preStart()
User overridable callback.long
previousClockTime()
void
previousClockTime_$eq(long x$1)
static Props
props(ReplicatorSettings settings)
Factory method for theProps
of theReplicator
actor.scala.Option<Cancellable>
pruningTask()
static Replicator.ReadLocal$
readLocal()
Java API: TheReadLocal
instancescala.PartialFunction<java.lang.Object,scala.runtime.BoxedUnit>
receive()
Scala API: This defines the initial actor behavior, it must return a partial function with the actor logic.void
receiveClockTick()
void
receiveDelete(Key<ReplicatedData> key, Replicator.WriteConsistency consistency, scala.Option<java.lang.Object> req)
void
receiveDeltaPropagation(UniqueAddress fromNode, boolean reply, scala.collection.immutable.Map<java.lang.String,akka.cluster.ddata.Replicator.Internal.Delta> deltas)
void
receiveDeltaPropagationTick()
void
receiveFlushChanges()
void
receiveGet(Key<ReplicatedData> key, Replicator.ReadConsistency consistency, scala.Option<java.lang.Object> req)
void
receiveGetKeyIds()
void
receiveGetReplicaCount()
void
receiveGossip(scala.collection.immutable.Map<java.lang.String,scala.Tuple2<akka.cluster.ddata.Replicator.Internal.DataEnvelope,java.lang.Object>> updatedData, boolean sendBack, scala.Option<java.lang.Object> fromSystemUid)
void
receiveGossipTick()
void
receiveMemberExiting(Member m)
void
receiveMemberJoining(Member m)
void
receiveMemberRemoved(Member m)
void
receiveMemberUp(Member m)
void
receiveMemberWeaklyUp(Member m)
void
receiveOtherMemberEvent(Member m)
void
receiveReachable(Member m)
void
receiveRead(java.lang.String key)
void
receiveReadRepair(java.lang.String key, akka.cluster.ddata.Replicator.Internal.DataEnvelope writeEnvelope)
void
receiveRemovedNodePruningTick()
void
receiveStatus(scala.collection.immutable.Map<java.lang.String,scala.Tuple2<ByteString,java.lang.Object>> otherDigests, int chunk, int totChunks, scala.Option<java.lang.Object> fromSystemUid)
void
receiveSubscribe(Key<ReplicatedData> key, ActorRef subscriber)
void
receiveTerminated(ActorRef ref)
void
receiveUnreachable(Member m)
void
receiveUnsubscribe(Key<ReplicatedData> key, ActorRef subscriber)
<A extends ReplicatedData>
voidreceiveUpdate(Key<ReplicatedData> key, scala.Function1<scala.Option<A>,A> modify, Replicator.WriteConsistency writeConsistency, scala.Option<java.lang.Object> req)
void
receiveWrite(java.lang.String key, akka.cluster.ddata.Replicator.Internal.DataEnvelope envelope)
scala.collection.immutable.Map<UniqueAddress,java.lang.Object>
removedNodes()
void
removedNodes_$eq(scala.collection.immutable.Map<UniqueAddress,java.lang.Object> x$1)
ActorSelection
replica(UniqueAddress node)
ActorRef
replyTo()
void
replyTo_$eq(ActorRef x$1)
scala.Option<UniqueAddress>
selectRandomNode(scala.collection.immutable.IndexedSeq<UniqueAddress> addresses)
ActorRef
self()
The 'self' field holds the ActorRef for this actor.Address
selfAddress()
scala.Some<java.lang.Object>
selfFromSystemUid()
UniqueAddress
selfUniqueAddress()
Serializer
serializer()
akka.cluster.ddata.Replicator.Internal.DataEnvelope
setData(java.lang.String key, akka.cluster.ddata.Replicator.Internal.DataEnvelope envelope)
long
statusCount()
void
statusCount_$eq(long x$1)
int
statusTotChunks()
void
statusTotChunks_$eq(int x$1)
scala.collection.mutable.HashMap<java.lang.String,scala.collection.mutable.Set<ActorRef>>
subscribers()
scala.collection.immutable.Map<java.lang.String,Key<ReplicatedData>>
subscriptionKeys()
void
subscriptionKeys_$eq(scala.collection.immutable.Map<java.lang.String,Key<ReplicatedData>> x$1)
OneForOneStrategy
supervisorStrategy()
User overridable definition the strategy to use for supervising child actors.scala.collection.immutable.Set<UniqueAddress>
unreachable()
void
unreachable_$eq(scala.collection.immutable.Set<UniqueAddress> x$1)
void
updateUsedTimestamp(java.lang.String key, long timestamp)
scala.collection.immutable.SortedSet<UniqueAddress>
weaklyUpNodes()
void
weaklyUpNodes_$eq(scala.collection.immutable.SortedSet<UniqueAddress> x$1)
scala.collection.mutable.HashMap<java.lang.String,scala.collection.mutable.Set<ActorRef>>
wildcardSubscribers()
scala.Option<akka.cluster.ddata.Replicator.Internal.DataEnvelope>
write(java.lang.String key, akka.cluster.ddata.Replicator.Internal.DataEnvelope writeEnvelope)
void
writeAndStore(java.lang.String key, akka.cluster.ddata.Replicator.Internal.DataEnvelope writeEnvelope, boolean reply)
static Replicator.WriteLocal$
writeLocal()
Java API: TheWriteLocal
instance-
Methods inherited from class java.lang.Object
clone, equals, finalize, getClass, hashCode, notify, notifyAll, toString, wait, wait, wait
-
Methods inherited from interface akka.actor.Actor
aroundPostRestart, aroundPostStop, aroundPreRestart, aroundPreStart, postRestart, preRestart, sender, unhandled
-
Methods inherited from interface akka.actor.ActorLogging
_log_$eq, log
-
-
-
-
Constructor Detail
-
Replicator
public Replicator(ReplicatorSettings settings)
-
-
Method Detail
-
props
public static Props props(ReplicatorSettings settings)
Factory method for theProps
of theReplicator
actor.
-
DefaultMajorityMinCap
public static int DefaultMajorityMinCap()
-
readLocal
public static Replicator.ReadLocal$ readLocal()
Java API: TheReadLocal
instance
-
writeLocal
public static Replicator.WriteLocal$ writeLocal()
Java API: TheWriteLocal
instance
-
getReplicaCount
public static Replicator.GetReplicaCount$ getReplicaCount()
Java API: TheGetReplicaCount
instance
-
flushChanges
public static Replicator.FlushChanges$ flushChanges()
Java API: TheFlushChanges
instance
-
context
public ActorContext context()
Description copied from interface:Actor
Scala API: Stores the context for this actor, including self, and sender. It is implicit to support operations such asforward
.WARNING: Only valid within the Actor itself, so do not close over it and publish it to other threads!
ActorContext
is the Scala API.getContext
returns aAbstractActor.ActorContext
, which is the Java API of the actor context.
-
self
public final ActorRef self()
Description copied from interface:Actor
The 'self' field holds the ActorRef for this actor. Can be used to send messages to itself:self ! message
-
akka$actor$Actor$_setter_$context_$eq
protected void akka$actor$Actor$_setter_$context_$eq(ActorContext x$1)
Description copied from interface:Actor
Scala API: Stores the context for this actor, including self, and sender. It is implicit to support operations such asforward
.WARNING: Only valid within the Actor itself, so do not close over it and publish it to other threads!
ActorContext
is the Scala API.getContext
returns aAbstractActor.ActorContext
, which is the Java API of the actor context.- Specified by:
akka$actor$Actor$_setter_$context_$eq
in interfaceActor
-
akka$actor$Actor$_setter_$self_$eq
protected final void akka$actor$Actor$_setter_$self_$eq(ActorRef x$1)
Description copied from interface:Actor
The 'self' field holds the ActorRef for this actor. Can be used to send messages to itself:self ! message
- Specified by:
akka$actor$Actor$_setter_$self_$eq
in interfaceActor
-
cluster
public Cluster cluster()
-
selfAddress
public Address selfAddress()
-
selfUniqueAddress
public UniqueAddress selfUniqueAddress()
-
selfFromSystemUid
public scala.Some<java.lang.Object> selfFromSystemUid()
-
gossipTask
public Cancellable gossipTask()
-
notifyTask
public Cancellable notifyTask()
-
pruningTask
public scala.Option<Cancellable> pruningTask()
-
clockTask
public Cancellable clockTask()
-
serializer
public Serializer serializer()
-
maxPruningDisseminationNanos
public long maxPruningDisseminationNanos()
-
expiryWildcards
public scala.collection.immutable.Map<java.lang.String,scala.concurrent.duration.FiniteDuration> expiryWildcards()
-
expiryEnabled
public boolean expiryEnabled()
-
hasDurableKeys
public boolean hasDurableKeys()
-
durable
public scala.collection.immutable.Set<java.lang.String> durable()
-
durableWildcards
public scala.collection.immutable.Set<java.lang.String> durableWildcards()
-
durableStore
public ActorRef durableStore()
-
deltaPropagationSelector
public java.lang.Object deltaPropagationSelector()
-
deltaPropagationTask
public scala.Option<Cancellable> deltaPropagationTask()
-
nodes
public scala.collection.immutable.SortedSet<UniqueAddress> nodes()
-
nodes_$eq
public void nodes_$eq(scala.collection.immutable.SortedSet<UniqueAddress> x$1)
-
membersByAge
public scala.collection.immutable.SortedSet<Member> membersByAge()
-
membersByAge_$eq
public void membersByAge_$eq(scala.collection.immutable.SortedSet<Member> x$1)
-
weaklyUpNodes
public scala.collection.immutable.SortedSet<UniqueAddress> weaklyUpNodes()
-
weaklyUpNodes_$eq
public void weaklyUpNodes_$eq(scala.collection.immutable.SortedSet<UniqueAddress> x$1)
-
joiningNodes
public scala.collection.immutable.SortedSet<UniqueAddress> joiningNodes()
-
joiningNodes_$eq
public void joiningNodes_$eq(scala.collection.immutable.SortedSet<UniqueAddress> x$1)
-
exitingNodes
public scala.collection.immutable.SortedSet<UniqueAddress> exitingNodes()
-
exitingNodes_$eq
public void exitingNodes_$eq(scala.collection.immutable.SortedSet<UniqueAddress> x$1)
-
removedNodes
public scala.collection.immutable.Map<UniqueAddress,java.lang.Object> removedNodes()
-
removedNodes_$eq
public void removedNodes_$eq(scala.collection.immutable.Map<UniqueAddress,java.lang.Object> x$1)
-
leader
public scala.collection.immutable.TreeSet<Member> leader()
-
leader_$eq
public void leader_$eq(scala.collection.immutable.TreeSet<Member> x$1)
-
isLeader
public boolean isLeader()
-
previousClockTime
public long previousClockTime()
-
previousClockTime_$eq
public void previousClockTime_$eq(long x$1)
-
allReachableClockTime
public long allReachableClockTime()
-
allReachableClockTime_$eq
public void allReachableClockTime_$eq(long x$1)
-
unreachable
public scala.collection.immutable.Set<UniqueAddress> unreachable()
-
unreachable_$eq
public void unreachable_$eq(scala.collection.immutable.Set<UniqueAddress> x$1)
-
dataEntries
public scala.collection.immutable.Map<java.lang.String,scala.Tuple3<akka.cluster.ddata.Replicator.Internal.DataEnvelope,ByteString,java.lang.Object>> dataEntries()
-
dataEntries_$eq
public void dataEntries_$eq(scala.collection.immutable.Map<java.lang.String,scala.Tuple3<akka.cluster.ddata.Replicator.Internal.DataEnvelope,ByteString,java.lang.Object>> x$1)
-
changed
public scala.collection.immutable.Set<java.lang.String> changed()
-
changed_$eq
public void changed_$eq(scala.collection.immutable.Set<java.lang.String> x$1)
-
statusCount
public long statusCount()
-
statusCount_$eq
public void statusCount_$eq(long x$1)
-
statusTotChunks
public int statusTotChunks()
-
statusTotChunks_$eq
public void statusTotChunks_$eq(int x$1)
-
fullStateGossipEnabled
public boolean fullStateGossipEnabled()
-
fullStateGossipEnabled_$eq
public void fullStateGossipEnabled_$eq(boolean x$1)
-
subscribers
public scala.collection.mutable.HashMap<java.lang.String,scala.collection.mutable.Set<ActorRef>> subscribers()
-
newSubscribers
public scala.collection.mutable.HashMap<java.lang.String,scala.collection.mutable.Set<ActorRef>> newSubscribers()
-
subscriptionKeys
public scala.collection.immutable.Map<java.lang.String,Key<ReplicatedData>> subscriptionKeys()
-
subscriptionKeys_$eq
public void subscriptionKeys_$eq(scala.collection.immutable.Map<java.lang.String,Key<ReplicatedData>> x$1)
-
wildcardSubscribers
public scala.collection.mutable.HashMap<java.lang.String,scala.collection.mutable.Set<ActorRef>> wildcardSubscribers()
-
replyTo
public ActorRef replyTo()
-
replyTo_$eq
public void replyTo_$eq(ActorRef x$1)
-
aroundReceive
protected void aroundReceive(scala.PartialFunction<java.lang.Object,scala.runtime.BoxedUnit> rcv, java.lang.Object msg)
Description copied from interface:Actor
INTERNAL API.Can be overridden to intercept calls to this actor's current behavior.
- Specified by:
aroundReceive
in interfaceActor
- Parameters:
rcv
- current behavior.msg
- current message.
-
preStart
public void preStart()
Description copied from interface:Actor
User overridable callback. Is called when an Actor is started. Actors are automatically started asynchronously when created. Empty default implementation.
-
postStop
public void postStop()
Description copied from interface:Actor
User overridable callback. Is called asynchronously after 'actor.stop()' is invoked. Empty default implementation.
-
matchingRole
public boolean matchingRole(Member m)
-
supervisorStrategy
public OneForOneStrategy supervisorStrategy()
Description copied from interface:Actor
User overridable definition the strategy to use for supervising child actors.- Specified by:
supervisorStrategy
in interfaceActor
-
receive
public scala.PartialFunction<java.lang.Object,scala.runtime.BoxedUnit> receive()
Description copied from interface:Actor
Scala API: This defines the initial actor behavior, it must return a partial function with the actor logic.
-
load
public scala.PartialFunction<java.lang.Object,scala.runtime.BoxedUnit> load()
-
normalReceive
public scala.PartialFunction<java.lang.Object,scala.runtime.BoxedUnit> normalReceive()
-
receiveGet
public void receiveGet(Key<ReplicatedData> key, Replicator.ReadConsistency consistency, scala.Option<java.lang.Object> req)
-
isLocalGet
public boolean isLocalGet(Replicator.ReadConsistency readConsistency)
-
receiveRead
public void receiveRead(java.lang.String key)
-
isLocalSender
public boolean isLocalSender()
-
receiveUpdate
public <A extends ReplicatedData> void receiveUpdate(Key<ReplicatedData> key, scala.Function1<scala.Option<A>,A> modify, Replicator.WriteConsistency writeConsistency, scala.Option<java.lang.Object> req)
-
isDurable
public boolean isDurable(java.lang.String key)
-
isLocalUpdate
public boolean isLocalUpdate(Replicator.WriteConsistency writeConsistency)
-
receiveWrite
public void receiveWrite(java.lang.String key, akka.cluster.ddata.Replicator.Internal.DataEnvelope envelope)
-
writeAndStore
public void writeAndStore(java.lang.String key, akka.cluster.ddata.Replicator.Internal.DataEnvelope writeEnvelope, boolean reply)
-
write
public scala.Option<akka.cluster.ddata.Replicator.Internal.DataEnvelope> write(java.lang.String key, akka.cluster.ddata.Replicator.Internal.DataEnvelope writeEnvelope)
-
receiveReadRepair
public void receiveReadRepair(java.lang.String key, akka.cluster.ddata.Replicator.Internal.DataEnvelope writeEnvelope)
-
receiveGetKeyIds
public void receiveGetKeyIds()
-
receiveDelete
public void receiveDelete(Key<ReplicatedData> key, Replicator.WriteConsistency consistency, scala.Option<java.lang.Object> req)
-
setData
public akka.cluster.ddata.Replicator.Internal.DataEnvelope setData(java.lang.String key, akka.cluster.ddata.Replicator.Internal.DataEnvelope envelope)
-
getDigest
public ByteString getDigest(java.lang.String key)
-
digest
public scala.Tuple2<ByteString,java.lang.Object> digest(akka.cluster.ddata.Replicator.Internal.DataEnvelope envelope)
- Returns:
- SHA-1 digest of the serialized data, and the size of the serialized data
-
getData
public scala.Option<akka.cluster.ddata.Replicator.Internal.DataEnvelope> getData(java.lang.String key)
-
getExpiryDuration
public scala.concurrent.duration.FiniteDuration getExpiryDuration(java.lang.String key)
-
getUsedTimestamp
public long getUsedTimestamp(java.lang.String key)
-
isExpired
public boolean isExpired(java.lang.String key)
-
isExpired
public boolean isExpired(java.lang.String key, long timestamp)
-
updateUsedTimestamp
public void updateUsedTimestamp(java.lang.String key, long timestamp)
-
getDeltaSeqNr
public long getDeltaSeqNr(java.lang.String key, UniqueAddress fromNode)
-
isNodeRemoved
public boolean isNodeRemoved(UniqueAddress node, scala.collection.Iterable<java.lang.String> keys)
-
receiveFlushChanges
public void receiveFlushChanges()
-
receiveDeltaPropagationTick
public void receiveDeltaPropagationTick()
-
receiveDeltaPropagation
public void receiveDeltaPropagation(UniqueAddress fromNode, boolean reply, scala.collection.immutable.Map<java.lang.String,akka.cluster.ddata.Replicator.Internal.Delta> deltas)
-
receiveGossipTick
public void receiveGossipTick()
-
gossipTo
public void gossipTo(UniqueAddress address)
-
selectRandomNode
public scala.Option<UniqueAddress> selectRandomNode(scala.collection.immutable.IndexedSeq<UniqueAddress> addresses)
-
replica
public ActorSelection replica(UniqueAddress node)
-
receiveStatus
public void receiveStatus(scala.collection.immutable.Map<java.lang.String,scala.Tuple2<ByteString,java.lang.Object>> otherDigests, int chunk, int totChunks, scala.Option<java.lang.Object> fromSystemUid)
-
receiveGossip
public void receiveGossip(scala.collection.immutable.Map<java.lang.String,scala.Tuple2<akka.cluster.ddata.Replicator.Internal.DataEnvelope,java.lang.Object>> updatedData, boolean sendBack, scala.Option<java.lang.Object> fromSystemUid)
-
receiveSubscribe
public void receiveSubscribe(Key<ReplicatedData> key, ActorRef subscriber)
-
receiveUnsubscribe
public void receiveUnsubscribe(Key<ReplicatedData> key, ActorRef subscriber)
-
hasSubscriber
public boolean hasSubscriber(ActorRef subscriber)
-
receiveTerminated
public void receiveTerminated(ActorRef ref)
-
receiveMemberJoining
public void receiveMemberJoining(Member m)
-
receiveMemberWeaklyUp
public void receiveMemberWeaklyUp(Member m)
-
receiveMemberUp
public void receiveMemberUp(Member m)
-
receiveMemberExiting
public void receiveMemberExiting(Member m)
-
receiveMemberRemoved
public void receiveMemberRemoved(Member m)
-
receiveOtherMemberEvent
public void receiveOtherMemberEvent(Member m)
-
receiveUnreachable
public void receiveUnreachable(Member m)
-
receiveReachable
public void receiveReachable(Member m)
-
receiveClockTick
public void receiveClockTick()
-
receiveRemovedNodePruningTick
public void receiveRemovedNodePruningTick()
-
collectRemovedNodes
public void collectRemovedNodes()
-
initRemovedNodePruning
public void initRemovedNodePruning()
-
performRemovedNodePruning
public void performRemovedNodePruning()
-
deleteObsoletePruningPerformed
public void deleteObsoletePruningPerformed()
-
receiveGetReplicaCount
public void receiveGetReplicaCount()
-
-