Convert mdsal submit() calls to commit()
[controller.git] / opendaylight / md-sal / sal-distributed-datastore / src / test / java / org / opendaylight / controller / cluster / datastore / DistributedDataStoreIntegrationTest.java
index 4a7a8134ad7892e9a6e472aefa1a4281248225b2..482021aa8b78cfc01a2d503ec5c592730d9217a8 100644 (file)
@@ -21,7 +21,7 @@ import akka.actor.ActorSystem;
 import akka.actor.Address;
 import akka.actor.AddressFromURIString;
 import akka.cluster.Cluster;
-import akka.testkit.JavaTestKit;
+import akka.testkit.javadsl.TestKit;
 import com.google.common.base.Optional;
 import com.google.common.base.Throwables;
 import com.google.common.collect.ImmutableMap;
@@ -40,7 +40,6 @@ import java.util.List;
 import java.util.concurrent.CountDownLatch;
 import java.util.concurrent.ExecutionException;
 import java.util.concurrent.TimeUnit;
-import java.util.concurrent.TimeoutException;
 import java.util.concurrent.atomic.AtomicReference;
 import org.junit.After;
 import org.junit.Before;
@@ -50,6 +49,7 @@ import org.junit.runners.Parameterized;
 import org.junit.runners.Parameterized.Parameter;
 import org.junit.runners.Parameterized.Parameters;
 import org.mockito.Mockito;
+import org.opendaylight.controller.cluster.access.client.RequestTimeoutException;
 import org.opendaylight.controller.cluster.databroker.ClientBackedDataStore;
 import org.opendaylight.controller.cluster.databroker.ConcurrentDOMDataBroker;
 import org.opendaylight.controller.cluster.datastore.exceptions.NoShardLeaderException;
@@ -59,7 +59,6 @@ import org.opendaylight.controller.cluster.datastore.messages.LocalShardFound;
 import org.opendaylight.controller.cluster.datastore.persisted.DatastoreSnapshot;
 import org.opendaylight.controller.cluster.datastore.persisted.MetadataShardDataTreeSnapshot;
 import org.opendaylight.controller.cluster.datastore.persisted.ShardSnapshotState;
-import org.opendaylight.controller.cluster.datastore.utils.MockDataChangeListener;
 import org.opendaylight.controller.cluster.datastore.utils.MockDataTreeChangeListener;
 import org.opendaylight.controller.cluster.raft.persisted.Snapshot;
 import org.opendaylight.controller.cluster.raft.utils.InMemoryJournal;
@@ -68,21 +67,19 @@ import org.opendaylight.controller.md.cluster.datastore.model.CarsModel;
 import org.opendaylight.controller.md.cluster.datastore.model.PeopleModel;
 import org.opendaylight.controller.md.cluster.datastore.model.SchemaContextHelper;
 import org.opendaylight.controller.md.cluster.datastore.model.TestModel;
-import org.opendaylight.controller.md.sal.common.api.data.AsyncDataBroker.DataChangeScope;
-import org.opendaylight.controller.md.sal.common.api.data.LogicalDatastoreType;
-import org.opendaylight.controller.md.sal.common.api.data.ReadFailedException;
-import org.opendaylight.controller.md.sal.common.api.data.TransactionChainClosedException;
-import org.opendaylight.controller.md.sal.common.api.data.TransactionChainListener;
-import org.opendaylight.controller.md.sal.common.api.data.TransactionCommitFailedException;
-import org.opendaylight.controller.md.sal.dom.api.DOMDataReadWriteTransaction;
-import org.opendaylight.controller.md.sal.dom.api.DOMDataWriteTransaction;
-import org.opendaylight.controller.md.sal.dom.api.DOMTransactionChain;
-import org.opendaylight.controller.sal.core.spi.data.DOMStore;
-import org.opendaylight.controller.sal.core.spi.data.DOMStoreReadTransaction;
-import org.opendaylight.controller.sal.core.spi.data.DOMStoreReadWriteTransaction;
-import org.opendaylight.controller.sal.core.spi.data.DOMStoreThreePhaseCommitCohort;
-import org.opendaylight.controller.sal.core.spi.data.DOMStoreTransactionChain;
-import org.opendaylight.controller.sal.core.spi.data.DOMStoreWriteTransaction;
+import org.opendaylight.mdsal.common.api.LogicalDatastoreType;
+import org.opendaylight.mdsal.common.api.ReadFailedException;
+import org.opendaylight.mdsal.common.api.TransactionChainClosedException;
+import org.opendaylight.mdsal.common.api.TransactionChainListener;
+import org.opendaylight.mdsal.dom.api.DOMDataTreeReadWriteTransaction;
+import org.opendaylight.mdsal.dom.api.DOMDataTreeWriteTransaction;
+import org.opendaylight.mdsal.dom.api.DOMTransactionChain;
+import org.opendaylight.mdsal.dom.spi.store.DOMStore;
+import org.opendaylight.mdsal.dom.spi.store.DOMStoreReadTransaction;
+import org.opendaylight.mdsal.dom.spi.store.DOMStoreReadWriteTransaction;
+import org.opendaylight.mdsal.dom.spi.store.DOMStoreThreePhaseCommitCohort;
+import org.opendaylight.mdsal.dom.spi.store.DOMStoreTransactionChain;
+import org.opendaylight.mdsal.dom.spi.store.DOMStoreWriteTransaction;
 import org.opendaylight.yangtools.concepts.ListenerRegistration;
 import org.opendaylight.yangtools.yang.data.api.YangInstanceIdentifier;
 import org.opendaylight.yangtools.yang.data.api.schema.ContainerNode;
@@ -90,7 +87,7 @@ import org.opendaylight.yangtools.yang.data.api.schema.MapEntryNode;
 import org.opendaylight.yangtools.yang.data.api.schema.MapNode;
 import org.opendaylight.yangtools.yang.data.api.schema.NormalizedNode;
 import org.opendaylight.yangtools.yang.data.api.schema.tree.DataTree;
-import org.opendaylight.yangtools.yang.data.api.schema.tree.TreeType;
+import org.opendaylight.yangtools.yang.data.api.schema.tree.DataTreeConfiguration;
 import org.opendaylight.yangtools.yang.data.impl.schema.ImmutableNodes;
 import org.opendaylight.yangtools.yang.data.impl.schema.builder.impl.ImmutableContainerNodeBuilder;
 import org.opendaylight.yangtools.yang.data.impl.schema.tree.InMemoryDataTreeFactory;
@@ -124,7 +121,7 @@ public class DistributedDataStoreIntegrationTest {
 
     @After
     public void tearDown() throws IOException {
-        JavaTestKit.shutdownActorSystem(system, null, Boolean.TRUE);
+        TestKit.shutdownActorSystem(system, Boolean.TRUE);
         system = null;
     }
 
@@ -554,7 +551,9 @@ public class DistributedDataStoreIntegrationTest {
                     txCohort.get().canCommit().get(5, TimeUnit.SECONDS);
                     fail("Expected NotInitializedException");
                 } catch (final Exception e) {
-                    Throwables.propagate(Throwables.getRootCause(e));
+                    final Throwable root = Throwables.getRootCause(e);
+                    Throwables.throwIfUnchecked(root);
+                    throw new RuntimeException(root);
                 } finally {
                     blockRecoveryLatch.countDown();
                 }
@@ -625,7 +624,9 @@ public class DistributedDataStoreIntegrationTest {
                         txReadFuture.get().checkedGet(5, TimeUnit.SECONDS);
                         fail("Expected NotInitializedException");
                     } catch (final ReadFailedException e) {
-                        Throwables.propagate(Throwables.getRootCause(e));
+                        final Throwable root = Throwables.getRootCause(e);
+                        Throwables.throwIfUnchecked(root);
+                        throw new RuntimeException(root);
                     } finally {
                         blockRecoveryLatch.countDown();
                     }
@@ -649,7 +650,7 @@ public class DistributedDataStoreIntegrationTest {
                 // The ShardManager uses the election timeout for FindPrimary so
                 // reset it low so it will timeout quickly.
                 datastoreContextBuilder.shardHeartbeatIntervalInMillis(100).shardElectionTimeoutFactor(1)
-                        .shardInitializationTimeout(200, TimeUnit.MILLISECONDS);
+                        .shardInitializationTimeout(200, TimeUnit.MILLISECONDS).frontendRequestTimeoutInSeconds(2);
 
                 try (AbstractDataStore dataStore = setupAbstractDataStore(
                         testParameter, testName, false, shardName)) {
@@ -699,14 +700,16 @@ public class DistributedDataStoreIntegrationTest {
                         // should have timed out and throw an appropriate
                         // exception cause.
                         try {
-                            txCohort.get().canCommit().get(5, TimeUnit.SECONDS);
+                            txCohort.get().canCommit().get(10, TimeUnit.SECONDS);
                             fail("Expected NoShardLeaderException");
                         } catch (final ExecutionException e) {
-                            assertTrue(Throwables.getRootCause(e) instanceof NoShardLeaderException);
-                            assertEquals(DistributedDataStore.class, testParameter);
-                        } catch (TimeoutException e) {
-                            // ClientBackedDataStore doesn't set cause to ExecutionException, future just time outs
-                            assertEquals(ClientBackedDataStore.class, testParameter);
+                            final String msg = "Unexpected exception: "
+                                    + Throwables.getStackTraceAsString(e.getCause());
+                            if (DistributedDataStore.class.equals(testParameter)) {
+                                assertTrue(Throwables.getRootCause(e) instanceof NoShardLeaderException);
+                            } else {
+                                assertTrue(msg, Throwables.getRootCause(e) instanceof RequestTimeoutException);
+                            }
                         }
                     } finally {
                         try {
@@ -927,25 +930,25 @@ public class DistributedDataStoreIntegrationTest {
                     final TransactionChainListener listener = Mockito.mock(TransactionChainListener.class);
                     DOMTransactionChain txChain = broker.createTransactionChain(listener);
 
-                    final List<CheckedFuture<Void, TransactionCommitFailedException>> futures = new ArrayList<>();
+                    final List<ListenableFuture<?>> futures = new ArrayList<>();
 
-                    final DOMDataWriteTransaction writeTx = txChain.newWriteOnlyTransaction();
+                    final DOMDataTreeWriteTransaction writeTx = txChain.newWriteOnlyTransaction();
                     writeTx.put(LogicalDatastoreType.CONFIGURATION, CarsModel.BASE_PATH, CarsModel.emptyContainer());
                     writeTx.put(LogicalDatastoreType.CONFIGURATION, CarsModel.CAR_LIST_PATH, CarsModel.newCarMapNode());
-                    futures.add(writeTx.submit());
+                    futures.add(writeTx.commit());
 
                     int numCars = 100;
                     for (int i = 0; i < numCars; i++) {
-                        final DOMDataReadWriteTransaction rwTx = txChain.newReadWriteTransaction();
+                        final DOMDataTreeReadWriteTransaction rwTx = txChain.newReadWriteTransaction();
 
                         rwTx.merge(LogicalDatastoreType.CONFIGURATION, CarsModel.newCarPath("car" + i),
                                 CarsModel.newCarEntry("car" + i, BigInteger.valueOf(20000)));
 
-                        futures.add(rwTx.submit());
+                        futures.add(rwTx.commit());
                     }
 
-                    for (final CheckedFuture<Void, TransactionCommitFailedException> f : futures) {
-                        f.checkedGet();
+                    for (final ListenableFuture<?> f : futures) {
+                        f.get(5, TimeUnit.SECONDS);
                     }
 
                     final Optional<NormalizedNode<?, ?>> optional = txChain.newReadOnlyTransaction()
@@ -1086,7 +1089,7 @@ public class DistributedDataStoreIntegrationTest {
                     final TransactionChainListener listener = Mockito.mock(TransactionChainListener.class);
                     final DOMTransactionChain txChain = broker.createTransactionChain(listener);
 
-                    final DOMDataReadWriteTransaction writeTx = txChain.newReadWriteTransaction();
+                    final DOMDataTreeReadWriteTransaction writeTx = txChain.newReadWriteTransaction();
 
                     writeTx.put(LogicalDatastoreType.CONFIGURATION, PeopleModel.BASE_PATH,
                             PeopleModel.emptyContainer());
@@ -1098,9 +1101,9 @@ public class DistributedDataStoreIntegrationTest {
                     writeTx.merge(LogicalDatastoreType.CONFIGURATION, CarsModel.BASE_PATH, invalidData);
 
                     try {
-                        writeTx.submit().checkedGet(5, TimeUnit.SECONDS);
+                        writeTx.commit().get(5, TimeUnit.SECONDS);
                         fail("Expected TransactionCommitFailedException");
-                    } catch (final TransactionCommitFailedException e) {
+                    } catch (final ExecutionException e) {
                         // Expected
                     }
 
@@ -1129,7 +1132,7 @@ public class DistributedDataStoreIntegrationTest {
                     final TransactionChainListener listener = Mockito.mock(TransactionChainListener.class);
                     final DOMTransactionChain txChain = broker.createTransactionChain(listener);
 
-                    final DOMDataReadWriteTransaction writeTx = txChain.newReadWriteTransaction();
+                    final DOMDataTreeWriteTransaction writeTx = txChain.newReadWriteTransaction();
 
                     writeTx.put(LogicalDatastoreType.CONFIGURATION, PeopleModel.BASE_PATH,
                             PeopleModel.emptyContainer());
@@ -1144,9 +1147,9 @@ public class DistributedDataStoreIntegrationTest {
                     // succeeds b/c deep validation is not
                     // done for put for performance reasons.
                     try {
-                        writeTx.submit().checkedGet(5, TimeUnit.SECONDS);
+                        writeTx.commit().get(5, TimeUnit.SECONDS);
                         fail("Expected TransactionCommitFailedException");
-                    } catch (final TransactionCommitFailedException e) {
+                    } catch (final ExecutionException e) {
                         // Expected
                     }
 
@@ -1160,59 +1163,6 @@ public class DistributedDataStoreIntegrationTest {
         };
     }
 
-    @Test
-    public void testChangeListenerRegistration() throws Exception {
-        new IntegrationTestKit(getSystem(), datastoreContextBuilder) {
-            {
-                try (AbstractDataStore dataStore = setupAbstractDataStore(
-                        testParameter, "testChangeListenerRegistration", "test-1")) {
-
-                    testWriteTransaction(dataStore, TestModel.TEST_PATH,
-                            ImmutableNodes.containerNode(TestModel.TEST_QNAME));
-
-                    final MockDataChangeListener listener = new MockDataChangeListener(1);
-
-                    final ListenerRegistration<MockDataChangeListener> listenerReg = dataStore
-                            .registerChangeListener(TestModel.TEST_PATH, listener, DataChangeScope.SUBTREE);
-
-                    assertNotNull("registerChangeListener returned null", listenerReg);
-
-                    IntegrationTestKit.verifyShardState(dataStore, "test-1",
-                        state -> assertEquals("getDataChangeListenerActors", 1,
-                                state.getDataChangeListenerActors().size()));
-
-                    // Wait for the initial notification
-                    listener.waitForChangeEvents(TestModel.TEST_PATH);
-                    listener.reset(2);
-
-                    // Write 2 updates.
-                    testWriteTransaction(dataStore, TestModel.OUTER_LIST_PATH,
-                            ImmutableNodes.mapNodeBuilder(TestModel.OUTER_LIST_QNAME).build());
-
-                    YangInstanceIdentifier listPath = YangInstanceIdentifier.builder(TestModel.OUTER_LIST_PATH)
-                            .nodeWithKey(TestModel.OUTER_LIST_QNAME, TestModel.ID_QNAME, 1).build();
-                    testWriteTransaction(dataStore, listPath,
-                            ImmutableNodes.mapEntry(TestModel.OUTER_LIST_QNAME, TestModel.ID_QNAME, 1));
-
-                    // Wait for the 2 updates.
-                    listener.waitForChangeEvents(TestModel.OUTER_LIST_PATH, listPath);
-                    listenerReg.close();
-
-                    IntegrationTestKit.verifyShardState(dataStore, "test-1",
-                        state -> assertEquals("getDataChangeListenerActors", 0,
-                                state.getDataChangeListenerActors().size()));
-
-                    testWriteTransaction(dataStore,
-                            YangInstanceIdentifier.builder(TestModel.OUTER_LIST_PATH)
-                                    .nodeWithKey(TestModel.OUTER_LIST_QNAME, TestModel.ID_QNAME, 2).build(),
-                            ImmutableNodes.mapEntry(TestModel.OUTER_LIST_QNAME, TestModel.ID_QNAME, 2));
-
-                    listener.expectNoMoreChanges("Received unexpected change after close");
-                }
-            }
-        };
-    }
-
     @Test
     public void testDataTreeChangeListenerRegistration() throws Exception {
         new IntegrationTestKit(getSystem(), datastoreContextBuilder) {
@@ -1276,8 +1226,8 @@ public class DistributedDataStoreIntegrationTest {
                         CarsModel.newCarsMapNode(CarsModel.newCarEntry("optima", BigInteger.valueOf(20000L)),
                                 CarsModel.newCarEntry("sportage", BigInteger.valueOf(30000L))));
 
-                DataTree dataTree = InMemoryDataTreeFactory.getInstance().create(TreeType.OPERATIONAL);
-                dataTree.setSchemaContext(SchemaContextHelper.full());
+                DataTree dataTree = new InMemoryDataTreeFactory().create(
+                    DataTreeConfiguration.DEFAULT_OPERATIONAL, SchemaContextHelper.full());
                 AbstractShardTest.writeToStore(dataTree, CarsModel.BASE_PATH, carsNode);
                 NormalizedNode<?, ?> root = AbstractShardTest.readStore(dataTree, YangInstanceIdentifier.EMPTY);
 
@@ -1285,8 +1235,8 @@ public class DistributedDataStoreIntegrationTest {
                         new ShardSnapshotState(new MetadataShardDataTreeSnapshot(root)),
                         Collections.emptyList(), 2, 1, 2, 1, 1, "member-1", null);
 
-                dataTree = InMemoryDataTreeFactory.getInstance().create(TreeType.OPERATIONAL);
-                dataTree.setSchemaContext(SchemaContextHelper.full());
+                dataTree = new InMemoryDataTreeFactory().create(DataTreeConfiguration.DEFAULT_OPERATIONAL,
+                    SchemaContextHelper.full());
 
                 final NormalizedNode<?, ?> peopleNode = PeopleModel.create();
                 AbstractShardTest.writeToStore(dataTree, PeopleModel.BASE_PATH, peopleNode);