X-Git-Url: https://git.opendaylight.org/gerrit/gitweb?a=blobdiff_plain;f=opendaylight%2Fmd-sal%2Fsal-distributed-datastore%2Fsrc%2Fmain%2Fjava%2Forg%2Fopendaylight%2Fcontroller%2Fcluster%2Fdatastore%2FShard.java;h=f7b3461d333f9c157dbeb06208eed276080c297d;hb=9b78e3a07fc3090565400eb2206a8e970f9a675b;hp=ee83ce2513dc1f9c3c8582820254ebdc3c5957c6;hpb=e970feb618c5e3793454f7f1a3974797a61c7c17;p=controller.git diff --git a/opendaylight/md-sal/sal-distributed-datastore/src/main/java/org/opendaylight/controller/cluster/datastore/Shard.java b/opendaylight/md-sal/sal-distributed-datastore/src/main/java/org/opendaylight/controller/cluster/datastore/Shard.java index ee83ce2513..f7b3461d33 100644 --- a/opendaylight/md-sal/sal-distributed-datastore/src/main/java/org/opendaylight/controller/cluster/datastore/Shard.java +++ b/opendaylight/md-sal/sal-distributed-datastore/src/main/java/org/opendaylight/controller/cluster/datastore/Shard.java @@ -39,6 +39,8 @@ import org.opendaylight.controller.cluster.datastore.messages.CommitTransaction; import org.opendaylight.controller.cluster.datastore.messages.CommitTransactionReply; import org.opendaylight.controller.cluster.datastore.messages.CreateTransaction; import org.opendaylight.controller.cluster.datastore.messages.CreateTransactionReply; +import org.opendaylight.controller.cluster.datastore.messages.DatastoreSnapshot; +import org.opendaylight.controller.cluster.datastore.messages.DatastoreSnapshot.ShardSnapshot; import org.opendaylight.controller.cluster.datastore.messages.ForwardedReadyTransaction; import org.opendaylight.controller.cluster.datastore.messages.GetShardDataTree; import org.opendaylight.controller.cluster.datastore.messages.PeerAddressResolved; @@ -61,12 +63,14 @@ import org.opendaylight.controller.cluster.raft.RaftActorSnapshotCohort; import org.opendaylight.controller.cluster.raft.RaftState; import org.opendaylight.controller.cluster.raft.base.messages.FollowerInitialSyncUpStatus; import org.opendaylight.controller.cluster.raft.messages.AppendEntriesReply; +import org.opendaylight.controller.cluster.raft.messages.ServerRemoved; import org.opendaylight.controller.cluster.raft.protobuff.client.messages.CompositeModificationByteStringPayload; import org.opendaylight.controller.cluster.raft.protobuff.client.messages.CompositeModificationPayload; import org.opendaylight.yangtools.yang.data.api.schema.tree.DataTree; import org.opendaylight.yangtools.yang.data.api.schema.tree.DataTreeCandidate; import org.opendaylight.yangtools.yang.data.api.schema.tree.DataValidationFailedException; import org.opendaylight.yangtools.yang.data.api.schema.tree.ModificationType; +import org.opendaylight.yangtools.yang.data.api.schema.tree.TreeType; import org.opendaylight.yangtools.yang.model.api.SchemaContext; import scala.concurrent.duration.Duration; import scala.concurrent.duration.FiniteDuration; @@ -114,18 +118,24 @@ public class Shard extends RaftActor { private final DataTreeChangeListenerSupport treeChangeSupport = new DataTreeChangeListenerSupport(this); private final DataChangeListenerSupport changeSupport = new DataChangeListenerSupport(this); + + private ShardSnapshot restoreFromSnapshot; + + + protected Shard(AbstractBuilder builder) { super(builder.getId().toString(), builder.getPeerAddresses(), Optional.of(builder.getDatastoreContext().getShardRaftConfig()), DataStoreVersions.CURRENT_VERSION); this.name = builder.getId().toString(); this.datastoreContext = builder.getDatastoreContext(); + this.restoreFromSnapshot = builder.getRestoreFromSnapshot(); setPersistence(datastoreContext.isPersistent()); LOG.info("Shard created : {}, persistent : {}", name, datastoreContext.isPersistent()); - store = new ShardDataTree(builder.getSchemaContext()); + store = new ShardDataTree(builder.getSchemaContext(), builder.getTreeType()); shardMBean = ShardMBeanFactory.getShardStatsMBean(name.toString(), datastoreContext.getDataStoreMXBeanType()); @@ -137,7 +147,7 @@ public class Shard extends RaftActor { commitCoordinator = new ShardCommitCoordinator(store, datastoreContext.getShardCommitQueueExpiryTimeoutInMillis(), - datastoreContext.getShardTransactionCommitQueueCapacity(), self(), LOG, this.name); + datastoreContext.getShardTransactionCommitQueueCapacity(), LOG, this.name); setTransactionCommitTimeout(); @@ -154,6 +164,7 @@ public class Shard extends RaftActor { snapshotCohort = new ShardSnapshotCohort(transactionActorFactory, store, LOG, this.name); + } private void setTransactionCommitTimeout() { @@ -218,8 +229,7 @@ public class Shard extends RaftActor { } else if (BatchedModifications.class.isInstance(message)) { handleBatchedModifications((BatchedModifications)message); } else if (message instanceof ForwardedReadyTransaction) { - commitCoordinator.handleForwardedReadyTransaction((ForwardedReadyTransaction) message, - getSender(), this); + handleForwardedReadyTransaction((ForwardedReadyTransaction) message); } else if (message instanceof ReadyLocalTransaction) { handleReadyLocalTransaction((ReadyLocalTransaction)message); } else if (CanCommitTransaction.SERIALIZABLE_CLASS.isInstance(message)) { @@ -251,8 +261,10 @@ public class Shard extends RaftActor { context().parent().tell(message, self()); } else if(GET_SHARD_MBEAN_MESSAGE.equals(message)){ sender().tell(getShardMBean(), self()); - } else if(message instanceof GetShardDataTree){ + } else if(message instanceof GetShardDataTree) { sender().tell(store.getDataTree(), self()); + } else if(message instanceof ServerRemoved){ + context().parent().forward(message, context()); } else { super.onReceiveCommand(message); } @@ -325,7 +337,7 @@ public class Shard extends RaftActor { applyModificationToState(cohortEntry.getReplySender(), cohortEntry.getTransactionID(), candidate); } else { Shard.this.persistData(cohortEntry.getReplySender(), cohortEntry.getTransactionID(), - DataTreeCandidatePayload.create(candidate)); + DataTreeCandidatePayload.create(candidate)); } } @@ -398,7 +410,7 @@ public class Shard extends RaftActor { commitCoordinator.handleCanCommit(canCommit.getTransactionID(), getSender(), this); } - private void noLeaderError(String errMessage, Object message) { + private void noLeaderError(String errMessage) { // TODO: rather than throwing an immediate exception, we could schedule a timer to try again to make // it more resilient in case we're in the process of electing a new leader. getSender().tell(new akka.actor.Status.Failure(new NoShardLeaderException(errMessage, persistenceId())), getSelf()); @@ -440,7 +452,7 @@ public class Shard extends RaftActor { LOG.debug("{}: Forwarding BatchedModifications to leader {}", persistenceId(), leader); leader.forward(batched, getContext()); } else { - noLeaderError("Could not commit transaction " + batched.getTransactionID(), batched); + noLeaderError("Could not commit transaction " + batched.getTransactionID()); } } } @@ -479,7 +491,27 @@ public class Shard extends RaftActor { message.setRemoteVersion(getCurrentBehavior().getLeaderPayloadVersion()); leader.forward(message, getContext()); } else { - noLeaderError("Could not commit transaction " + message.getTransactionID(), message); + noLeaderError("Could not commit transaction " + message.getTransactionID()); + } + } + } + + private void handleForwardedReadyTransaction(ForwardedReadyTransaction forwardedReady) { + if (isLeader()) { + failIfIsolatedLeader(getSender()); + + commitCoordinator.handleForwardedReadyTransaction(forwardedReady, getSender(), this); + } else { + ActorSelection leader = getLeader(); + if (leader != null) { + LOG.debug("{}: Forwarding ForwardedReadyTransaction to leader {}", persistenceId(), leader); + + ReadyLocalTransaction readyLocal = new ReadyLocalTransaction(forwardedReady.getTransactionID(), + forwardedReady.getTransaction().getSnapshot(), forwardedReady.isDoImmediateCommit()); + readyLocal.setRemoteVersion(getCurrentBehavior().getLeaderPayloadVersion()); + leader.forward(readyLocal, getContext()); + } else { + noLeaderError("Could not commit transaction " + forwardedReady.getTransactionID()); } } } @@ -585,11 +617,14 @@ public class Shard extends RaftActor { @Override @Nonnull protected RaftActorRecoveryCohort getRaftActorRecoveryCohort() { - return new ShardRecoveryCoordinator(store, store.getSchemaContext(), persistenceId(), LOG); + return new ShardRecoveryCoordinator(store, store.getSchemaContext(), + restoreFromSnapshot != null ? restoreFromSnapshot.getSnapshot() : null, persistenceId(), LOG); } @Override protected void onRecoveryComplete() { + restoreFromSnapshot = null; + //notify shard manager getContext().parent().tell(new ActorInitialized(), getSelf()); @@ -712,6 +747,7 @@ public class Shard extends RaftActor { private Map peerAddresses = Collections.emptyMap(); private DatastoreContext datastoreContext; private SchemaContext schemaContext; + private DatastoreSnapshot.ShardSnapshot restoreFromSnapshot; private volatile boolean sealed; protected AbstractBuilder(Class shardClass) { @@ -751,6 +787,12 @@ public class Shard extends RaftActor { return self(); } + public T restoreFromSnapshot(DatastoreSnapshot.ShardSnapshot restoreFromSnapshot) { + checkSealed(); + this.restoreFromSnapshot = restoreFromSnapshot; + return self(); + } + public ShardIdentifier getId() { return id; } @@ -767,6 +809,21 @@ public class Shard extends RaftActor { return schemaContext; } + public DatastoreSnapshot.ShardSnapshot getRestoreFromSnapshot() { + return restoreFromSnapshot; + } + + public TreeType getTreeType() { + switch (datastoreContext.getLogicalStoreType()) { + case CONFIGURATION: + return TreeType.CONFIGURATION; + case OPERATIONAL: + return TreeType.OPERATIONAL; + } + + throw new IllegalStateException("Unhandled logical store type " + datastoreContext.getLogicalStoreType()); + } + protected void verify() { Preconditions.checkNotNull(id, "id should not be null"); Preconditions.checkNotNull(peerAddresses, "peerAddresses should not be null");