BUG-5280: centralize ShardSnapshot operations
[controller.git] / opendaylight / md-sal / sal-distributed-datastore / src / main / java / org / opendaylight / controller / cluster / datastore / ShardSnapshotCohort.java
index c59085d61c57961feb915077e996262b92e2ce17..7812d70917757083dce3dace6d52b14d2c1994ec 100644 (file)
@@ -7,15 +7,21 @@
  */
 package org.opendaylight.controller.cluster.datastore;
 
+import akka.actor.ActorContext;
 import akka.actor.ActorRef;
+import com.google.common.base.Preconditions;
+import java.io.IOException;
+import java.util.Optional;
 import java.util.concurrent.ExecutionException;
-import org.opendaylight.controller.cluster.datastore.identifiers.ShardTransactionIdentifier;
-import org.opendaylight.controller.cluster.datastore.messages.CreateSnapshot;
-import org.opendaylight.controller.cluster.datastore.utils.SerializationUtils;
+import org.opendaylight.controller.cluster.access.concepts.ClientIdentifier;
+import org.opendaylight.controller.cluster.access.concepts.FrontendIdentifier;
+import org.opendaylight.controller.cluster.access.concepts.FrontendType;
+import org.opendaylight.controller.cluster.access.concepts.LocalHistoryIdentifier;
+import org.opendaylight.controller.cluster.access.concepts.MemberName;
+import org.opendaylight.controller.cluster.access.concepts.TransactionIdentifier;
+import org.opendaylight.controller.cluster.datastore.actors.ShardSnapshotActor;
+import org.opendaylight.controller.cluster.datastore.persisted.ShardDataTreeSnapshot;
 import org.opendaylight.controller.cluster.raft.RaftActorSnapshotCohort;
-import org.opendaylight.controller.md.sal.dom.store.impl.InMemoryDOMDataStore;
-import org.opendaylight.controller.sal.core.spi.data.DOMStoreThreePhaseCommitCohort;
-import org.opendaylight.controller.sal.core.spi.data.DOMStoreWriteTransaction;
 import org.opendaylight.yangtools.yang.data.api.YangInstanceIdentifier;
 import org.opendaylight.yangtools.yang.data.api.schema.NormalizedNode;
 import org.slf4j.Logger;
@@ -26,69 +32,88 @@ import org.slf4j.Logger;
  * @author Thomas Pantelis
  */
 class ShardSnapshotCohort implements RaftActorSnapshotCohort {
+    private static final FrontendType SNAPSHOT_APPLY = FrontendType.forName("snapshot-apply");
 
-    private static final YangInstanceIdentifier DATASTORE_ROOT = YangInstanceIdentifier.builder().build();
-
-    private int createSnapshotTransactionCounter;
-    private final ShardTransactionActorFactory transactionActorFactory;
-    private final InMemoryDOMDataStore store;
-    private final Logger log;
+    private final LocalHistoryIdentifier applyHistoryId;
+    private final ActorRef snapshotActor;
+    private final ShardDataTree store;
     private final String logId;
+    private final Logger log;
 
-    ShardSnapshotCohort(ShardTransactionActorFactory transactionActorFactory, InMemoryDOMDataStore store,
-            Logger log, String logId) {
-        this.transactionActorFactory = transactionActorFactory;
-        this.store = store;
+    private long applyCounter;
+
+    private ShardSnapshotCohort(final LocalHistoryIdentifier applyHistoryId, final ActorRef snapshotActor,
+            final ShardDataTree store, final Logger log, final String logId) {
+        this.applyHistoryId = Preconditions.checkNotNull(applyHistoryId);
+        this.snapshotActor = Preconditions.checkNotNull(snapshotActor);
+        this.store = Preconditions.checkNotNull(store);
         this.log = log;
         this.logId = logId;
     }
 
-    @Override
-    public void createSnapshot(ActorRef actorRef) {
-        // Create a transaction actor. We are really going to treat the transaction as a worker
-        // so that this actor does not get block building the snapshot. THe transaction actor will
-        // after processing the CreateSnapshot message.
-
-        ShardTransactionIdentifier transactionID = new ShardTransactionIdentifier(
-                "createSnapshot" + ++createSnapshotTransactionCounter);
+    static ShardSnapshotCohort create(final ActorContext actorContext, final MemberName memberName,
+            final ShardDataTree store, final Logger log, final String logId) {
+        final LocalHistoryIdentifier applyHistoryId = new LocalHistoryIdentifier(ClientIdentifier.create(
+            FrontendIdentifier.create(memberName, SNAPSHOT_APPLY), 0), 0);
+        final String snapshotActorName = "shard-" + memberName.getName() + ':' + "snapshot-read";
 
-        ActorRef createSnapshotTransaction = transactionActorFactory.newShardTransaction(
-                TransactionProxy.TransactionType.READ_ONLY, transactionID, "", DataStoreVersions.CURRENT_VERSION);
+        // Create a snapshot actor. This actor will act as a worker to offload snapshot serialization for all
+        // requests.
+        final ActorRef snapshotActor = actorContext.actorOf(ShardSnapshotActor.props(), snapshotActorName);
 
-        createSnapshotTransaction.tell(CreateSnapshot.INSTANCE, actorRef);
+        return new ShardSnapshotCohort(applyHistoryId, snapshotActor, store, log, logId);
     }
 
     @Override
-    public void applySnapshot(byte[] snapshotBytes) {
-        // Since this will be done only on Recovery or when this actor is a Follower
-        // we can safely commit everything in here. We not need to worry about event notifications
-        // as they would have already been disabled on the follower
-
-        log.info("{}: Applying snapshot", logId);
+    public void createSnapshot(final ActorRef actorRef) {
+        // Forward the request to the snapshot actor
+        ShardSnapshotActor.requestSnapshot(snapshotActor, store.takeRecoverySnapshot(), actorRef);
+    }
 
+    private void deserializeAndApplySnapshot(final byte[] snapshotBytes) {
+        final ShardDataTreeSnapshot snapshot;
         try {
-            DOMStoreWriteTransaction transaction = store.newWriteOnlyTransaction();
+            snapshot = ShardDataTreeSnapshot.deserialize(snapshotBytes);
+        } catch (IOException e) {
+            log.error("{}: Failed to deserialize snapshot", logId, e);
+            return;
+        }
 
-            NormalizedNode<?, ?> node = SerializationUtils.deserializeNormalizedNode(snapshotBytes);
+        try {
+            final ReadWriteShardDataTreeTransaction transaction = store.newReadWriteTransaction(
+                new TransactionIdentifier(applyHistoryId, applyCounter++));
 
             // delete everything first
-            transaction.delete(DATASTORE_ROOT);
+            transaction.getSnapshot().delete(YangInstanceIdentifier.EMPTY);
+
+            final Optional<NormalizedNode<?, ?>> maybeNode = snapshot.getRootNode();
+            if (maybeNode.isPresent()) {
+                // Add everything from the remote node back
+                transaction.getSnapshot().write(YangInstanceIdentifier.EMPTY, maybeNode.get());
+            }
 
-            // Add everything from the remote node back
-            transaction.write(DATASTORE_ROOT, node);
-            syncCommitTransaction(transaction);
-        } catch (InterruptedException | ExecutionException e) {
+            store.applyRecoveryTransaction(transaction);
+        } catch (Exception e) {
             log.error("{}: An exception occurred when applying snapshot", logId, e);
-        } finally {
-            log.info("{}: Done applying snapshot", logId);
         }
 
     }
 
-    void syncCommitTransaction(final DOMStoreWriteTransaction transaction)
+    void syncCommitTransaction(final ReadWriteShardDataTreeTransaction transaction)
             throws ExecutionException, InterruptedException {
-        DOMStoreThreePhaseCommitCohort commitCohort = transaction.ready();
+        ShardDataTreeCohort commitCohort = store.finishTransaction(transaction);
         commitCohort.preCommit().get();
         commitCohort.commit().get();
     }
+
+    @Override
+    public void applySnapshot(final byte[] snapshotBytes) {
+        // Since this will be done only on Recovery or when this actor is a Follower
+        // we can safely commit everything in here. We not need to worry about event notifications
+        // as they would have already been disabled on the follower
+
+        log.info("{}: Applying snapshot", logId);
+        deserializeAndApplySnapshot(snapshotBytes);
+        log.info("{}: Done applying snapshot", logId);
+    }
 }