X-Git-Url: https://git.opendaylight.org/gerrit/gitweb?p=controller.git;a=blobdiff_plain;f=opendaylight%2Fmd-sal%2Fsal-distributed-datastore%2Fsrc%2Fmain%2Fjava%2Forg%2Fopendaylight%2Fcontroller%2Fcluster%2Fdatastore%2Fshardmanager%2FShardManager.java;h=187d2522de9e7df810a878b8c4605a3e3a44e2eb;hp=d3d8ce39c9a108f0b497b942f9e6e5368f3c6a21;hb=20a32e6459fd1e27e7669bf1ebc7742b96787b94;hpb=aa307bc6c06d9bcf8e877553af9babc95c42c39b diff --git a/opendaylight/md-sal/sal-distributed-datastore/src/main/java/org/opendaylight/controller/cluster/datastore/shardmanager/ShardManager.java b/opendaylight/md-sal/sal-distributed-datastore/src/main/java/org/opendaylight/controller/cluster/datastore/shardmanager/ShardManager.java index d3d8ce39c9..187d2522de 100644 --- a/opendaylight/md-sal/sal-distributed-datastore/src/main/java/org/opendaylight/controller/cluster/datastore/shardmanager/ShardManager.java +++ b/opendaylight/md-sal/sal-distributed-datastore/src/main/java/org/opendaylight/controller/cluster/datastore/shardmanager/ShardManager.java @@ -51,6 +51,7 @@ import java.util.function.Consumer; import java.util.function.Supplier; import org.opendaylight.controller.cluster.access.concepts.MemberName; import org.opendaylight.controller.cluster.common.actor.AbstractUntypedPersistentActorWithMetering; +import org.opendaylight.controller.cluster.common.actor.Dispatchers; import org.opendaylight.controller.cluster.datastore.AbstractDataStore; import org.opendaylight.controller.cluster.datastore.ClusterWrapper; import org.opendaylight.controller.cluster.datastore.DatastoreContext; @@ -73,6 +74,8 @@ import org.opendaylight.controller.cluster.datastore.messages.CreateShard; import org.opendaylight.controller.cluster.datastore.messages.FindLocalShard; import org.opendaylight.controller.cluster.datastore.messages.FindPrimary; import org.opendaylight.controller.cluster.datastore.messages.FlipShardMembersVotingStatus; +import org.opendaylight.controller.cluster.datastore.messages.GetShardRole; +import org.opendaylight.controller.cluster.datastore.messages.GetShardRoleReply; import org.opendaylight.controller.cluster.datastore.messages.LocalPrimaryShardFound; import org.opendaylight.controller.cluster.datastore.messages.LocalShardFound; import org.opendaylight.controller.cluster.datastore.messages.LocalShardNotFound; @@ -85,7 +88,7 @@ import org.opendaylight.controller.cluster.datastore.messages.UpdateSchemaContex import org.opendaylight.controller.cluster.datastore.persisted.DatastoreSnapshot; import org.opendaylight.controller.cluster.datastore.persisted.ShardManagerSnapshot; import org.opendaylight.controller.cluster.datastore.utils.ClusterUtils; -import org.opendaylight.controller.cluster.datastore.utils.Dispatchers; +import org.opendaylight.controller.cluster.datastore.utils.CompositeOnComplete; import org.opendaylight.controller.cluster.datastore.utils.PrimaryShardInfoFutureCache; import org.opendaylight.controller.cluster.notifications.RegisterRoleChangeListener; import org.opendaylight.controller.cluster.notifications.RegisterRoleChangeListenerReply; @@ -167,7 +170,7 @@ class ShardManager extends AbstractUntypedPersistentActorWithMetering { private final Set shardReplicaOperationsInProgress = new HashSet<>(); - private final Map> shardActorStoppingFutures = new HashMap<>(); + private final Map> shardActorsStopping = new HashMap<>(); private final String persistenceId; private final AbstractDataStore dataStore; @@ -175,7 +178,7 @@ class ShardManager extends AbstractUntypedPersistentActorWithMetering { private ListenerRegistration configListenerReg = null; private PrefixedShardConfigUpdateHandler configUpdateHandler; - ShardManager(AbstractShardManagerCreator builder) { + ShardManager(final AbstractShardManagerCreator builder) { this.cluster = builder.getCluster(); this.configuration = builder.getConfiguration(); this.datastoreContextFactory = builder.getDatastoreContextFactory(); @@ -220,7 +223,7 @@ class ShardManager extends AbstractUntypedPersistentActorWithMetering { } @Override - public void handleCommand(Object message) throws Exception { + public void handleCommand(final Object message) throws Exception { if (message instanceof FindPrimary) { findPrimary((FindPrimary)message); } else if (message instanceof FindLocalShard) { @@ -295,6 +298,8 @@ class ShardManager extends AbstractUntypedPersistentActorWithMetering { onShutDown(); } else if (message instanceof GetLocalShardIds) { onGetLocalShardIds(); + } else if (message instanceof GetShardRole) { + onGetShardRole((GetShardRole) message); } else if (message instanceof RunnableMessage) { ((RunnableMessage)message).run(); } else if (message instanceof DeleteSnapshotsFailure) { @@ -311,10 +316,27 @@ class ShardManager extends AbstractUntypedPersistentActorWithMetering { } } + private void onGetShardRole(final GetShardRole message) { + LOG.debug("{}: onGetShardRole for shard: {}", persistenceId(), message.getName()); + + final String name = message.getName(); + + final ShardInformation shardInformation = localShards.get(name); + + if (shardInformation == null) { + LOG.info("{}: no shard information for {} found", persistenceId(), name); + getSender().tell(new Status.Failure( + new IllegalArgumentException("Shard with name " + name + " not present.")), ActorRef.noSender()); + return; + } + + getSender().tell(new GetShardRoleReply(shardInformation.getRole()), ActorRef.noSender()); + } + private void onInitConfigListener() { LOG.debug("{}: Initializing config listener on {}", persistenceId(), cluster.getCurrentMemberName()); - final org.opendaylight.mdsal.common.api.LogicalDatastoreType type = + final org.opendaylight.mdsal.common.api.LogicalDatastoreType datastoreType = org.opendaylight.mdsal.common.api.LogicalDatastoreType .valueOf(datastoreContextFactory.getBaseDatastoreContext().getLogicalStoreType().name()); @@ -323,7 +345,7 @@ class ShardManager extends AbstractUntypedPersistentActorWithMetering { } configUpdateHandler = new PrefixedShardConfigUpdateHandler(self(), cluster.getCurrentMemberName()); - configUpdateHandler.initListener(dataStore, type); + configUpdateHandler.initListener(dataStore, datastoreType); } private void onShutDown() { @@ -346,7 +368,7 @@ class ShardManager extends AbstractUntypedPersistentActorWithMetering { combinedFutures.onComplete(new OnComplete>() { @Override - public void onComplete(Throwable failure, Iterable results) { + public void onComplete(final Throwable failure, final Iterable results) { LOG.debug("{}: All shards shutdown - sending PoisonPill to self", persistenceId()); self().tell(PoisonPill.getInstance(), self()); @@ -369,15 +391,15 @@ class ShardManager extends AbstractUntypedPersistentActorWithMetering { }, dispatcher); } - private void onWrappedShardResponse(WrappedShardResponse message) { + private void onWrappedShardResponse(final WrappedShardResponse message) { if (message.getResponse() instanceof RemoveServerReply) { onRemoveServerReply(getSender(), message.getShardId(), (RemoveServerReply) message.getResponse(), message.getLeaderPath()); } } - private void onRemoveServerReply(ActorRef originalSender, ShardIdentifier shardId, RemoveServerReply replyMsg, - String leaderPath) { + private void onRemoveServerReply(final ActorRef originalSender, final ShardIdentifier shardId, + final RemoveServerReply replyMsg, final String leaderPath) { shardReplicaOperationsInProgress.remove(shardId.getShardName()); LOG.debug("{}: Received {} for shard {}", persistenceId(), replyMsg, shardId.getShardName()); @@ -417,7 +439,7 @@ class ShardManager extends AbstractUntypedPersistentActorWithMetering { futureObj.onComplete(new OnComplete() { @Override - public void onComplete(Throwable failure, Object response) { + public void onComplete(final Throwable failure, final Object response) { if (failure != null) { shardReplicaOperationsInProgress.remove(shardName); String msg = String.format("RemoveServer request to leader %s for shard %s failed", @@ -435,8 +457,8 @@ class ShardManager extends AbstractUntypedPersistentActorWithMetering { }, new Dispatchers(context().system().dispatchers()).getDispatcher(Dispatchers.DispatcherType.Client)); } - private void removeShardReplica(RemoveShardReplica contextMessage, final String shardName, final String primaryPath, - final ActorRef sender) { + private void removeShardReplica(final RemoveShardReplica contextMessage, final String shardName, + final String primaryPath, final ActorRef sender) { if (isShardReplicaOperationInProgress(shardName, sender)) { return; } @@ -457,7 +479,7 @@ class ShardManager extends AbstractUntypedPersistentActorWithMetering { futureObj.onComplete(new OnComplete() { @Override - public void onComplete(Throwable failure, Object response) { + public void onComplete(final Throwable failure, final Object response) { if (failure != null) { shardReplicaOperationsInProgress.remove(shardName); String msg = String.format("RemoveServer request to leader %s for shard %s failed", @@ -475,7 +497,7 @@ class ShardManager extends AbstractUntypedPersistentActorWithMetering { }, new Dispatchers(context().system().dispatchers()).getDispatcher(Dispatchers.DispatcherType.Client)); } - private void onShardReplicaRemoved(ServerRemoved message) { + private void onShardReplicaRemoved(final ServerRemoved message) { removeShard(new ShardIdentifier.Builder().fromShardIdString(message.getServerId()).build()); } @@ -490,24 +512,37 @@ class ShardManager extends AbstractUntypedPersistentActorWithMetering { final ActorRef shardActor = shardInformation.getActor(); if (shardActor != null) { - LOG.debug("{} : Sending Shutdown to Shard actor {}", persistenceId(), shardActor); - FiniteDuration duration = shardInformation.getDatastoreContext().getShardRaftConfig() - .getElectionTimeOutInterval().$times(3); - final Future stopFuture = Patterns.gracefulStop(shardActor, duration, Shutdown.INSTANCE); - shardActorStoppingFutures.put(shardName, stopFuture); - stopFuture.onComplete(new OnComplete() { + long timeoutInMS = Math.max(shardInformation.getDatastoreContext().getShardRaftConfig() + .getElectionTimeOutInterval().$times(3).toMillis(), 10000); + + LOG.debug("{} : Sending Shutdown to Shard actor {} with {} ms timeout", persistenceId(), shardActor, + timeoutInMS); + + final Future stopFuture = Patterns.gracefulStop(shardActor, + FiniteDuration.apply(timeoutInMS, TimeUnit.MILLISECONDS), Shutdown.INSTANCE); + + final CompositeOnComplete onComplete = new CompositeOnComplete() { @Override - public void onComplete(Throwable failure, Boolean result) { + public void onComplete(final Throwable failure, final Boolean result) { if (failure == null) { LOG.debug("{} : Successfully shut down Shard actor {}", persistenceId(), shardActor); } else { LOG.warn("{}: Failed to shut down Shard actor {}", persistenceId(), shardActor, failure); } - self().tell((RunnableMessage) () -> shardActorStoppingFutures.remove(shardName), - ActorRef.noSender()); + self().tell((RunnableMessage) () -> { + // At any rate, invalidate primaryShardInfo cache + primaryShardInfoCache.remove(shardName); + + shardActorsStopping.remove(shardName); + notifyOnCompleteTasks(failure, result); + }, ActorRef.noSender()); } - }, new Dispatchers(context().system().dispatchers()).getDispatcher(Dispatchers.DispatcherType.Client)); + }; + + shardActorsStopping.put(shardName, onComplete); + stopFuture.onComplete(onComplete, new Dispatchers(context().system().dispatchers()) + .getDispatcher(Dispatchers.DispatcherType.Client)); } LOG.debug("{} : Local Shard replica for shard {} has been removed", persistenceId(), shardName); @@ -544,7 +579,7 @@ class ShardManager extends AbstractUntypedPersistentActorWithMetering { } @SuppressWarnings("checkstyle:IllegalCatch") - private void onCreateShard(CreateShard createShard) { + private void onCreateShard(final CreateShard createShard) { LOG.debug("{}: onCreateShard: {}", persistenceId(), createShard); Object reply; @@ -594,26 +629,27 @@ class ShardManager extends AbstractUntypedPersistentActorWithMetering { } private boolean isPreviousShardActorStopInProgress(final String shardName, final Object messageToDefer) { - final Future stopFuture = shardActorStoppingFutures.get(shardName); - if (stopFuture == null) { + final CompositeOnComplete stopOnComplete = shardActorsStopping.get(shardName); + if (stopOnComplete == null) { return false; } - LOG.debug("{} : Stop is in progress for shard {} - adding Future callback to defer {}", persistenceId(), + LOG.debug("{} : Stop is in progress for shard {} - adding OnComplete callback to defer {}", persistenceId(), shardName, messageToDefer); final ActorRef sender = getSender(); - stopFuture.onComplete(new OnComplete() { + stopOnComplete.addOnComplete(new OnComplete() { @Override - public void onComplete(Throwable failure, Boolean result) { + public void onComplete(final Throwable failure, final Boolean result) { LOG.debug("{} : Stop complete for shard {} - re-queing {}", persistenceId(), shardName, messageToDefer); self().tell(messageToDefer, sender); } - }, new Dispatchers(context().system().dispatchers()).getDispatcher(Dispatchers.DispatcherType.Client)); + }); return true; } - private void doCreatePrefixShard(PrefixShardConfiguration config, ShardIdentifier shardId, String shardName) { + private void doCreatePrefixShard(final PrefixShardConfiguration config, final ShardIdentifier shardId, + final String shardName) { configuration.addPrefixShardConfiguration(config); final Builder builder = newShardDatastoreContextBuilder(shardName); @@ -633,7 +669,8 @@ class ShardManager extends AbstractUntypedPersistentActorWithMetering { localShards.put(info.getShardName(), info); if (schemaContext != null) { - info.setActor(newShardActor(schemaContext, info)); + info.setSchemaContext(schemaContext); + info.setActor(newShardActor(info)); } } @@ -694,16 +731,17 @@ class ShardManager extends AbstractUntypedPersistentActorWithMetering { localShards.put(info.getShardName(), info); if (schemaContext != null) { - info.setActor(newShardActor(schemaContext, info)); + info.setSchemaContext(schemaContext); + info.setActor(newShardActor(info)); } } - private DatastoreContext.Builder newShardDatastoreContextBuilder(String shardName) { + private DatastoreContext.Builder newShardDatastoreContextBuilder(final String shardName) { return DatastoreContext.newBuilderFrom(datastoreContextFactory.getShardDatastoreContext(shardName)) .shardPeerAddressResolver(peerAddressResolver); } - private DatastoreContext newShardDatastoreContext(String shardName) { + private DatastoreContext newShardDatastoreContext(final String shardName) { return newShardDatastoreContextBuilder(shardName).build(); } @@ -716,7 +754,7 @@ class ShardManager extends AbstractUntypedPersistentActorWithMetering { } } - private void onLeaderStateChanged(ShardLeaderStateChanged leaderStateChanged) { + private void onLeaderStateChanged(final ShardLeaderStateChanged leaderStateChanged) { LOG.info("{}: Received LeaderStateChanged message: {}", persistenceId(), leaderStateChanged); ShardInformation shardInformation = findShardInformation(leaderStateChanged.getMemberId()); @@ -733,7 +771,7 @@ class ShardManager extends AbstractUntypedPersistentActorWithMetering { } } - private void onShardNotInitializedTimeout(ShardNotInitializedTimeout message) { + private void onShardNotInitializedTimeout(final ShardNotInitializedTimeout message) { ShardInformation shardInfo = message.getShardInfo(); LOG.debug("{}: Received ShardNotInitializedTimeout message for shard {}", persistenceId(), @@ -750,7 +788,7 @@ class ShardManager extends AbstractUntypedPersistentActorWithMetering { } } - private void onFollowerInitialSyncStatus(FollowerInitialSyncUpStatus status) { + private void onFollowerInitialSyncStatus(final FollowerInitialSyncUpStatus status) { LOG.info("{} Received follower initial sync status for {} status sync done {}", persistenceId(), status.getName(), status.isInitialSyncDone()); @@ -764,7 +802,7 @@ class ShardManager extends AbstractUntypedPersistentActorWithMetering { } - private void onRoleChangeNotification(RoleChangeNotification roleChanged) { + private void onRoleChangeNotification(final RoleChangeNotification roleChanged) { LOG.info("{}: Received role changed for {} from {} to {}", persistenceId(), roleChanged.getMemberId(), roleChanged.getOldRole(), roleChanged.getNewRole()); @@ -777,7 +815,7 @@ class ShardManager extends AbstractUntypedPersistentActorWithMetering { } - private ShardInformation findShardInformation(String memberId) { + private ShardInformation findShardInformation(final String memberId) { for (ShardInformation info : localShards.values()) { if (info.getShardId().toString().equals(memberId)) { return info; @@ -807,11 +845,12 @@ class ShardManager extends AbstractUntypedPersistentActorWithMetering { return true; } - private void onActorInitialized(Object message) { + private void onActorInitialized(final Object message) { final ActorRef sender = getSender(); if (sender == null) { - return; //why is a non-actor sending this message? Just ignore. + // why is a non-actor sending this message? Just ignore. + return; } String actorName = sender.path().name(); @@ -828,7 +867,7 @@ class ShardManager extends AbstractUntypedPersistentActorWithMetering { markShardAsInitialized(shardId.getShardName()); } - private void markShardAsInitialized(String shardName) { + private void markShardAsInitialized(final String shardName) { LOG.debug("{}: Initializing shard [{}]", persistenceId(), shardName); ShardInformation shardInformation = localShards.get(shardName); @@ -840,7 +879,7 @@ class ShardManager extends AbstractUntypedPersistentActorWithMetering { } @Override - protected void handleRecover(Object message) throws Exception { + protected void handleRecover(final Object message) throws Exception { if (message instanceof RecoveryCompleted) { onRecoveryCompleted(); } else if (message instanceof SnapshotOffer) { @@ -864,8 +903,8 @@ class ShardManager extends AbstractUntypedPersistentActorWithMetering { createLocalShards(); } - private void sendResponse(ShardInformation shardInformation, boolean doWait, - boolean wantShardReady, final Supplier messageSupplier) { + private void sendResponse(final ShardInformation shardInformation, final boolean doWait, + final boolean wantShardReady, final Supplier messageSupplier) { if (!shardInformation.isShardInitialized() || wantShardReady && !shardInformation.isShardReadyWithLeaderId()) { if (doWait) { final ActorRef sender = getSender(); @@ -913,11 +952,11 @@ class ShardManager extends AbstractUntypedPersistentActorWithMetering { getSender().tell(messageSupplier.get(), getSelf()); } - private static NoShardLeaderException createNoShardLeaderException(ShardIdentifier shardId) { + private static NoShardLeaderException createNoShardLeaderException(final ShardIdentifier shardId) { return new NoShardLeaderException(null, shardId.toString()); } - private static NotInitializedException createNotInitializedException(ShardIdentifier shardId) { + private static NotInitializedException createNotInitializedException(final ShardIdentifier shardId) { return new NotInitializedException(String.format( "Found primary shard %s but it's not initialized yet. Please try again later", shardId)); } @@ -927,7 +966,7 @@ class ShardManager extends AbstractUntypedPersistentActorWithMetering { return MemberName.forName(member.roles().iterator().next()); } - private void memberRemoved(ClusterEvent.MemberRemoved message) { + private void memberRemoved(final ClusterEvent.MemberRemoved message) { MemberName memberName = memberToName(message.member()); LOG.info("{}: Received MemberRemoved: memberName: {}, address: {}", persistenceId(), memberName, @@ -940,7 +979,7 @@ class ShardManager extends AbstractUntypedPersistentActorWithMetering { } } - private void memberExited(ClusterEvent.MemberExited message) { + private void memberExited(final ClusterEvent.MemberExited message) { MemberName memberName = memberToName(message.member()); LOG.info("{}: Received MemberExited: memberName: {}, address: {}", persistenceId(), memberName, @@ -953,7 +992,7 @@ class ShardManager extends AbstractUntypedPersistentActorWithMetering { } } - private void memberUp(ClusterEvent.MemberUp message) { + private void memberUp(final ClusterEvent.MemberUp message) { MemberName memberName = memberToName(message.member()); LOG.info("{}: Received MemberUp: memberName: {}, address: {}", persistenceId(), memberName, @@ -962,12 +1001,12 @@ class ShardManager extends AbstractUntypedPersistentActorWithMetering { memberUp(memberName, message.member().address()); } - private void memberUp(MemberName memberName, Address address) { + private void memberUp(final MemberName memberName, final Address address) { addPeerAddress(memberName, address); checkReady(); } - private void memberWeaklyUp(MemberWeaklyUp message) { + private void memberWeaklyUp(final MemberWeaklyUp message) { MemberName memberName = memberToName(message.member()); LOG.info("{}: Received MemberWeaklyUp: memberName: {}, address: {}", persistenceId(), memberName, @@ -976,7 +1015,7 @@ class ShardManager extends AbstractUntypedPersistentActorWithMetering { memberUp(memberName, message.member().address()); } - private void addPeerAddress(MemberName memberName, Address address) { + private void addPeerAddress(final MemberName memberName, final Address address) { peerAddressResolver.addPeerAddress(memberName, address); for (ShardInformation info : localShards.values()) { @@ -988,7 +1027,7 @@ class ShardManager extends AbstractUntypedPersistentActorWithMetering { } } - private void memberReachable(ClusterEvent.ReachableMember message) { + private void memberReachable(final ClusterEvent.ReachableMember message) { MemberName memberName = memberToName(message.member()); LOG.info("Received ReachableMember: memberName {}, address: {}", memberName, message.member().address()); @@ -997,7 +1036,7 @@ class ShardManager extends AbstractUntypedPersistentActorWithMetering { markMemberAvailable(memberName); } - private void memberUnreachable(ClusterEvent.UnreachableMember message) { + private void memberUnreachable(final ClusterEvent.UnreachableMember message) { MemberName memberName = memberToName(message.member()); LOG.info("Received UnreachableMember: memberName {}, address: {}", memberName, message.member().address()); @@ -1005,11 +1044,9 @@ class ShardManager extends AbstractUntypedPersistentActorWithMetering { } private void markMemberUnavailable(final MemberName memberName) { - final String memberStr = memberName.getName(); for (ShardInformation info : localShards.values()) { String leaderId = info.getLeaderId(); - // XXX: why are we using String#contains() here? - if (leaderId != null && leaderId.contains(memberStr)) { + if (leaderId != null && ShardIdentifier.fromShardIdString(leaderId).getMemberName().equals(memberName)) { LOG.debug("Marking Leader {} as unavailable.", leaderId); info.setLeaderAvailable(false); @@ -1021,11 +1058,9 @@ class ShardManager extends AbstractUntypedPersistentActorWithMetering { } private void markMemberAvailable(final MemberName memberName) { - final String memberStr = memberName.getName(); for (ShardInformation info : localShards.values()) { String leaderId = info.getLeaderId(); - // XXX: why are we using String#contains() here? - if (leaderId != null && leaderId.contains(memberStr)) { + if (leaderId != null && ShardIdentifier.fromShardIdString(leaderId).getMemberName().equals(memberName)) { LOG.debug("Marking Leader {} as available.", leaderId); info.setLeaderAvailable(true); } @@ -1034,7 +1069,7 @@ class ShardManager extends AbstractUntypedPersistentActorWithMetering { } } - private void onDatastoreContextFactory(DatastoreContextFactory factory) { + private void onDatastoreContextFactory(final DatastoreContextFactory factory) { datastoreContextFactory = factory; for (ShardInformation info : localShards.values()) { info.setDatastoreContext(newShardDatastoreContext(info.getShardName()), getSelf()); @@ -1090,12 +1125,25 @@ class ShardManager extends AbstractUntypedPersistentActorWithMetering { private void updateSchemaContext(final Object message) { schemaContext = ((UpdateSchemaContext) message).getSchemaContext(); - LOG.debug("Got updated SchemaContext: # of modules {}", schemaContext.getAllModuleIdentifiers().size()); + LOG.debug("Got updated SchemaContext: # of modules {}", schemaContext.getModules().size()); for (ShardInformation info : localShards.values()) { + info.setSchemaContext(schemaContext); + if (info.getActor() == null) { LOG.debug("Creating Shard {}", info.getShardId()); - info.setActor(newShardActor(schemaContext, info)); + info.setActor(newShardActor(info)); + // Update peer address for every existing peer memeber to avoid missing sending + // PeerAddressResolved and PeerUp to this shard while UpdateSchemaContext comes after MemberUp. + String shardName = info.getShardName(); + for (MemberName memberName : peerAddressResolver.getPeerMembers()) { + String peerId = getShardIdentifier(memberName, shardName).toString() ; + String peerAddress = peerAddressResolver.getShardActorAddress(shardName, memberName); + info.updatePeerAddress(peerId, peerAddress, getSelf()); + info.peerUp(memberName, peerId, getSelf()); + LOG.debug("{}: updated peer {} on member {} with address {} on shard {} whose actor address is {}", + persistenceId(), peerId, memberName, peerAddress, info.getShardId(), info.getActor()); + } } else { info.getActor().tell(message, getSelf()); } @@ -1108,12 +1156,12 @@ class ShardManager extends AbstractUntypedPersistentActorWithMetering { } @VisibleForTesting - protected ActorRef newShardActor(final SchemaContext shardSchemaContext, final ShardInformation info) { - return getContext().actorOf(info.newProps(shardSchemaContext).withDispatcher(shardDispatcherPath), + protected ActorRef newShardActor(final ShardInformation info) { + return getContext().actorOf(info.newProps().withDispatcher(shardDispatcherPath), info.getShardId().toString()); } - private void findPrimary(FindPrimary message) { + private void findPrimary(final FindPrimary message) { LOG.debug("{}: In findPrimary: {}", persistenceId(), message); final String shardName = message.getShardName(); @@ -1170,7 +1218,7 @@ class ShardManager extends AbstractUntypedPersistentActorWithMetering { Future futureObj = ask(getSelf(), new FindPrimary(shardName, true), findPrimaryTimeout); futureObj.onComplete(new OnComplete() { @Override - public void onComplete(Throwable failure, Object response) { + public void onComplete(final Throwable failure, final Object response) { if (failure != null) { handler.onFailure(failure); } else { @@ -1194,7 +1242,7 @@ class ShardManager extends AbstractUntypedPersistentActorWithMetering { * @param shardName the shard name * @return a b */ - private ShardIdentifier getShardIdentifier(MemberName memberName, String shardName) { + private ShardIdentifier getShardIdentifier(final MemberName memberName, final String shardName) { return peerAddressResolver.getShardIdentifier(memberName, shardName); } @@ -1212,7 +1260,8 @@ class ShardManager extends AbstractUntypedPersistentActorWithMetering { } } - restoreFromSnapshot = null; // null out to GC + // null out to GC + restoreFromSnapshot = null; for (String shardName : memberShardNames) { ShardIdentifier shardId = getShardIdentifier(memberName, shardName); @@ -1231,7 +1280,7 @@ class ShardManager extends AbstractUntypedPersistentActorWithMetering { * * @param shardName the shard name */ - private Map getPeerAddresses(String shardName) { + private Map getPeerAddresses(final String shardName) { final Collection members = configuration.getMembersFromShardName(shardName); return getPeerAddresses(shardName, members); } @@ -1300,7 +1349,7 @@ class ShardManager extends AbstractUntypedPersistentActorWithMetering { findPrimary(shardName, new AutoFindPrimaryFailureResponseHandler(getSender(), shardName, persistenceId(), getSelf()) { @Override - public void onRemotePrimaryShardFound(RemotePrimaryShardFound response) { + public void onRemotePrimaryShardFound(final RemotePrimaryShardFound response) { final RunnableMessage runnable = (RunnableMessage) () -> addPrefixShard(getShardName(), message.getShardPrefix(), response, getSender()); if (!isPreviousShardActorStopInProgress(getShardName(), runnable)) { @@ -1309,7 +1358,7 @@ class ShardManager extends AbstractUntypedPersistentActorWithMetering { } @Override - public void onLocalPrimaryFound(LocalPrimaryShardFound message) { + public void onLocalPrimaryFound(final LocalPrimaryShardFound message) { sendLocalReplicaAlreadyExistsReply(getShardName(), getTargetActor()); } }); @@ -1340,7 +1389,7 @@ class ShardManager extends AbstractUntypedPersistentActorWithMetering { findPrimary(shardName, new AutoFindPrimaryFailureResponseHandler(getSender(), shardName, persistenceId(), getSelf()) { @Override - public void onRemotePrimaryShardFound(RemotePrimaryShardFound response) { + public void onRemotePrimaryShardFound(final RemotePrimaryShardFound response) { final RunnableMessage runnable = (RunnableMessage) () -> addShard(getShardName(), response, getSender()); if (!isPreviousShardActorStopInProgress(getShardName(), runnable)) { @@ -1349,13 +1398,13 @@ class ShardManager extends AbstractUntypedPersistentActorWithMetering { } @Override - public void onLocalPrimaryFound(LocalPrimaryShardFound message) { + public void onLocalPrimaryFound(final LocalPrimaryShardFound message) { sendLocalReplicaAlreadyExistsReply(getShardName(), getTargetActor()); } }); } - private void sendLocalReplicaAlreadyExistsReply(String shardName, ActorRef sender) { + private void sendLocalReplicaAlreadyExistsReply(final String shardName, final ActorRef sender) { String msg = String.format("Local shard %s already exists", shardName); LOG.debug("{}: {}", persistenceId(), msg); sender.tell(new Status.Failure(new AlreadyExistsException(msg)), getSelf()); @@ -1384,8 +1433,9 @@ class ShardManager extends AbstractUntypedPersistentActorWithMetering { shardInfo = new ShardInformation(shardName, shardId, getPeerAddresses(shardName), datastoreContext, Shard.builder(), peerAddressResolver); shardInfo.setActiveMember(false); + shardInfo.setSchemaContext(schemaContext); localShards.put(shardName, shardInfo); - shardInfo.setActor(newShardActor(schemaContext, shardInfo)); + shardInfo.setActor(newShardActor(shardInfo)); } else { removeShardOnFailure = false; shardInfo = existingShardInfo; @@ -1414,8 +1464,9 @@ class ShardManager extends AbstractUntypedPersistentActorWithMetering { shardInfo = new ShardInformation(shardName, shardId, getPeerAddresses(shardName), datastoreContext, Shard.builder(), peerAddressResolver); shardInfo.setActiveMember(false); + shardInfo.setSchemaContext(schemaContext); localShards.put(shardName, shardInfo); - shardInfo.setActor(newShardActor(schemaContext, shardInfo)); + shardInfo.setActor(newShardActor(shardInfo)); } else { removeShardOnFailure = false; shardInfo = existingShardInfo; @@ -1444,7 +1495,7 @@ class ShardManager extends AbstractUntypedPersistentActorWithMetering { futureObj.onComplete(new OnComplete() { @Override - public void onComplete(Throwable failure, Object addServerResponse) { + public void onComplete(final Throwable failure, final Object addServerResponse) { if (failure != null) { LOG.debug("{}: AddServer request to {} for {} failed", persistenceId(), response.getPrimaryPath(), shardName, failure); @@ -1460,8 +1511,8 @@ class ShardManager extends AbstractUntypedPersistentActorWithMetering { }, new Dispatchers(context().system().dispatchers()).getDispatcher(Dispatchers.DispatcherType.Client)); } - private void onAddServerFailure(String shardName, String message, Throwable failure, ActorRef sender, - boolean removeShardOnFailure) { + private void onAddServerFailure(final String shardName, final String message, final Throwable failure, + final ActorRef sender, final boolean removeShardOnFailure) { shardReplicaOperationsInProgress.remove(shardName); if (removeShardOnFailure) { @@ -1475,8 +1526,8 @@ class ShardManager extends AbstractUntypedPersistentActorWithMetering { new RuntimeException(message, failure)), getSelf()); } - private void onAddServerReply(ShardInformation shardInfo, AddServerReply replyMsg, ActorRef sender, - String leaderPath, boolean removeShardOnFailure) { + private void onAddServerReply(final ShardInformation shardInfo, final AddServerReply replyMsg, + final ActorRef sender, final String leaderPath, final boolean removeShardOnFailure) { String shardName = shardInfo.getShardName(); shardReplicaOperationsInProgress.remove(shardName); @@ -1504,29 +1555,23 @@ class ShardManager extends AbstractUntypedPersistentActorWithMetering { } } - private static Exception getServerChangeException(Class serverChange, ServerChangeStatus serverChangeStatus, - String leaderPath, ShardIdentifier shardId) { - Exception failure; + private static Exception getServerChangeException(final Class serverChange, + final ServerChangeStatus serverChangeStatus, final String leaderPath, final ShardIdentifier shardId) { switch (serverChangeStatus) { case TIMEOUT: - failure = new TimeoutException(String.format( + return new TimeoutException(String.format( "The shard leader %s timed out trying to replicate the initial data to the new shard %s." + "Possible causes - there was a problem replicating the data or shard leadership changed " + "while replicating the shard data", leaderPath, shardId.getShardName())); - break; case NO_LEADER: - failure = createNoShardLeaderException(shardId); - break; + return createNoShardLeaderException(shardId); case NOT_SUPPORTED: - failure = new UnsupportedOperationException(String.format("%s request is not supported for shard %s", + return new UnsupportedOperationException(String.format("%s request is not supported for shard %s", serverChange.getSimpleName(), shardId.getShardName())); - break; default : - failure = new RuntimeException(String.format( - "%s request to leader %s for shard %s failed with status %s", + return new RuntimeException(String.format("%s request to leader %s for shard %s failed with status %s", serverChange.getSimpleName(), leaderPath, shardId.getShardName(), serverChangeStatus)); } - return failure; } private void onRemoveShardReplica(final RemoveShardReplica shardReplicaMsg) { @@ -1535,12 +1580,12 @@ class ShardManager extends AbstractUntypedPersistentActorWithMetering { findPrimary(shardReplicaMsg.getShardName(), new AutoFindPrimaryFailureResponseHandler(getSender(), shardReplicaMsg.getShardName(), persistenceId(), getSelf()) { @Override - public void onRemotePrimaryShardFound(RemotePrimaryShardFound response) { + public void onRemotePrimaryShardFound(final RemotePrimaryShardFound response) { doRemoveShardReplicaAsync(response.getPrimaryPath()); } @Override - public void onLocalPrimaryFound(LocalPrimaryShardFound response) { + public void onLocalPrimaryFound(final LocalPrimaryShardFound response) { doRemoveShardReplicaAsync(response.getPrimaryPath()); } @@ -1561,12 +1606,12 @@ class ShardManager extends AbstractUntypedPersistentActorWithMetering { findPrimary(shardName, new AutoFindPrimaryFailureResponseHandler(getSender(), shardName, persistenceId(), getSelf()) { @Override - public void onRemotePrimaryShardFound(RemotePrimaryShardFound response) { + public void onRemotePrimaryShardFound(final RemotePrimaryShardFound response) { doRemoveShardReplicaAsync(response.getPrimaryPath()); } @Override - public void onLocalPrimaryFound(LocalPrimaryShardFound response) { + public void onLocalPrimaryFound(final LocalPrimaryShardFound response) { doRemoveShardReplicaAsync(response.getPrimaryPath()); } @@ -1595,7 +1640,7 @@ class ShardManager extends AbstractUntypedPersistentActorWithMetering { return currentSnapshot; } - private void applyShardManagerSnapshot(ShardManagerSnapshot snapshot) { + private void applyShardManagerSnapshot(final ShardManagerSnapshot snapshot) { currentSnapshot = snapshot; LOG.debug("{}: onSnapshotOffer: {}", persistenceId(), currentSnapshot); @@ -1619,7 +1664,7 @@ class ShardManager extends AbstractUntypedPersistentActorWithMetering { } } - private void onSaveSnapshotSuccess(SaveSnapshotSuccess successMessage) { + private void onSaveSnapshotSuccess(final SaveSnapshotSuccess successMessage) { LOG.debug("{} saved ShardManager snapshot successfully. Deleting the prev snapshot if available", persistenceId()); deleteSnapshots(new SnapshotSelectionCriteria(scala.Long.MaxValue(), successMessage.metadata().timestamp() - 1, @@ -1643,7 +1688,7 @@ class ShardManager extends AbstractUntypedPersistentActorWithMetering { localShardFound.getPath(), getSender())); } - private void onFlipShardMembersVotingStatus(FlipShardMembersVotingStatus flipMembersVotingStatus) { + private void onFlipShardMembersVotingStatus(final FlipShardMembersVotingStatus flipMembersVotingStatus) { LOG.debug("{}: onFlipShardMembersVotingStatus: {}", persistenceId(), flipMembersVotingStatus); ActorRef sender = getSender(); @@ -1654,7 +1699,7 @@ class ShardManager extends AbstractUntypedPersistentActorWithMetering { future.onComplete(new OnComplete() { @Override - public void onComplete(Throwable failure, Object response) { + public void onComplete(final Throwable failure, final Object response) { if (failure != null) { sender.tell(new Status.Failure(new RuntimeException( String.format("Failed to access local shard %s", shardName), failure)), self()); @@ -1678,7 +1723,7 @@ class ShardManager extends AbstractUntypedPersistentActorWithMetering { } - private void findLocalShard(FindLocalShard message) { + private void findLocalShard(final FindLocalShard message) { LOG.debug("{}: findLocalShard : {}", persistenceId(), message.getShardName()); final ShardInformation shardInformation = localShards.get(message.getShardName()); @@ -1703,7 +1748,7 @@ class ShardManager extends AbstractUntypedPersistentActorWithMetering { Future futureObj = ask(getSelf(), new FindLocalShard(shardName, true), findLocalTimeout); futureObj.onComplete(new OnComplete() { @Override - public void onComplete(Throwable failure, Object response) { + public void onComplete(final Throwable failure, final Object response) { if (failure != null) { LOG.debug("{}: Received failure from FindLocalShard for shard {}", persistenceId, shardName, failure); @@ -1729,7 +1774,7 @@ class ShardManager extends AbstractUntypedPersistentActorWithMetering { }, new Dispatchers(context().system().dispatchers()).getDispatcher(Dispatchers.DispatcherType.Client)); } - private void changeShardMembersVotingStatus(ChangeServersVotingStatus changeServersVotingStatus, + private void changeShardMembersVotingStatus(final ChangeServersVotingStatus changeServersVotingStatus, final String shardName, final ActorRef shardActorRef, final ActorRef sender) { if (isShardReplicaOperationInProgress(shardName, sender)) { return; @@ -1748,7 +1793,7 @@ class ShardManager extends AbstractUntypedPersistentActorWithMetering { futureObj.onComplete(new OnComplete() { @Override - public void onComplete(Throwable failure, Object response) { + public void onComplete(final Throwable failure, final Object response) { shardReplicaOperationsInProgress.remove(shardName); if (failure != null) { String msg = String.format("ChangeServersVotingStatus request to local shard %s failed", @@ -1785,8 +1830,8 @@ class ShardManager extends AbstractUntypedPersistentActorWithMetering { String leaderPath; boolean removeShardOnFailure; - ForwardedAddServerReply(ShardInformation shardInfo, AddServerReply addServerReply, String leaderPath, - boolean removeShardOnFailure) { + ForwardedAddServerReply(final ShardInformation shardInfo, final AddServerReply addServerReply, + final String leaderPath, final boolean removeShardOnFailure) { this.shardInfo = shardInfo; this.addServerReply = addServerReply; this.leaderPath = leaderPath; @@ -1800,8 +1845,8 @@ class ShardManager extends AbstractUntypedPersistentActorWithMetering { Throwable failure; boolean removeShardOnFailure; - ForwardedAddServerFailure(String shardName, String failureMessage, Throwable failure, - boolean removeShardOnFailure) { + ForwardedAddServerFailure(final String shardName, final String failureMessage, final Throwable failure, + final boolean removeShardOnFailure) { this.shardName = shardName; this.failureMessage = failureMessage; this.failure = failure; @@ -1813,7 +1858,7 @@ class ShardManager extends AbstractUntypedPersistentActorWithMetering { private final Runnable replyRunnable; private Cancellable timeoutSchedule; - OnShardInitialized(Runnable replyRunnable) { + OnShardInitialized(final Runnable replyRunnable) { this.replyRunnable = replyRunnable; } @@ -1825,13 +1870,13 @@ class ShardManager extends AbstractUntypedPersistentActorWithMetering { return timeoutSchedule; } - void setTimeoutSchedule(Cancellable timeoutSchedule) { + void setTimeoutSchedule(final Cancellable timeoutSchedule) { this.timeoutSchedule = timeoutSchedule; } } static class OnShardReady extends OnShardInitialized { - OnShardReady(Runnable replyRunnable) { + OnShardReady(final Runnable replyRunnable) { super(replyRunnable); } } @@ -1891,8 +1936,8 @@ class ShardManager extends AbstractUntypedPersistentActorWithMetering { * @param persistenceId The persistenceId for the ShardManager * @param shardManagerActor The ShardManager actor which triggered the call to FindPrimary */ - protected AutoFindPrimaryFailureResponseHandler(ActorRef targetActor, String shardName, String persistenceId, - ActorRef shardManagerActor) { + protected AutoFindPrimaryFailureResponseHandler(final ActorRef targetActor, final String shardName, + final String persistenceId, final ActorRef shardManagerActor) { this.targetActor = Preconditions.checkNotNull(targetActor); this.shardName = Preconditions.checkNotNull(shardName); this.persistenceId = Preconditions.checkNotNull(persistenceId); @@ -1908,14 +1953,14 @@ class ShardManager extends AbstractUntypedPersistentActorWithMetering { } @Override - public void onFailure(Throwable failure) { + public void onFailure(final Throwable failure) { LOG.debug("{}: Received failure from FindPrimary for shard {}", persistenceId, shardName, failure); targetActor.tell(new Status.Failure(new RuntimeException( String.format("Failed to find leader for shard %s", shardName), failure)), shardManagerActor); } @Override - public void onUnknownResponse(Object response) { + public void onUnknownResponse(final Object response) { String msg = String.format("Failed to find leader for shard %s: received response: %s", shardName, response); LOG.debug("{}: {}", persistenceId, msg); @@ -1932,7 +1977,7 @@ class ShardManager extends AbstractUntypedPersistentActorWithMetering { private final Object response; private final String leaderPath; - WrappedShardResponse(ShardIdentifier shardId, Object response, String leaderPath) { + WrappedShardResponse(final ShardIdentifier shardId, final Object response, final String leaderPath) { this.shardId = shardId; this.response = response; this.leaderPath = leaderPath; @@ -1956,7 +2001,8 @@ class ShardManager extends AbstractUntypedPersistentActorWithMetering { private final ShardInformation shardInfo; private final OnShardInitialized onShardInitialized; - ShardNotInitializedTimeout(ShardInformation shardInfo, OnShardInitialized onShardInitialized, ActorRef sender) { + ShardNotInitializedTimeout(final ShardInformation shardInfo, final OnShardInitialized onShardInitialized, + final ActorRef sender) { this.sender = sender; this.shardInfo = shardInfo; this.onShardInitialized = onShardInitialized;