Merge "Fixed build-failure because of undeclared transtive dependency."
authorTony Tkacik <ttkacik@cisco.com>
Sun, 8 Mar 2015 14:15:02 +0000 (14:15 +0000)
committerGerrit Code Review <gerrit@opendaylight.org>
Sun, 8 Mar 2015 14:15:02 +0000 (14:15 +0000)
43 files changed:
opendaylight/md-sal/sal-distributed-datastore/src/main/java/org/opendaylight/controller/cluster/datastore/DistributedDataStore.java
opendaylight/md-sal/sal-distributed-datastore/src/main/java/org/opendaylight/controller/cluster/datastore/DistributedDataStoreFactory.java
opendaylight/md-sal/sal-distributed-datastore/src/main/java/org/opendaylight/controller/cluster/datastore/Shard.java
opendaylight/md-sal/sal-distributed-datastore/src/main/java/org/opendaylight/controller/cluster/datastore/ShardManager.java
opendaylight/md-sal/sal-distributed-datastore/src/main/java/org/opendaylight/controller/cluster/datastore/utils/MessageTracker.java
opendaylight/md-sal/sal-distributed-datastore/src/test/java/org/opendaylight/controller/cluster/datastore/DistributedDataStoreTest.java
opendaylight/md-sal/sal-distributed-datastore/src/test/java/org/opendaylight/controller/cluster/datastore/ShardManagerTest.java
opendaylight/md-sal/sal-distributed-datastore/src/test/java/org/opendaylight/controller/cluster/datastore/ShardTest.java
opendaylight/netconf/mdsal-netconf-connector/pom.xml
opendaylight/netconf/mdsal-netconf-connector/src/test/java/org/opendaylight/controller/netconf/mdsal/connector/ops/NetconfMDSalMappingTest.java [new file with mode: 0644]
opendaylight/netconf/mdsal-netconf-connector/src/test/resources/messages/mapping/commit.xml [new file with mode: 0644]
opendaylight/netconf/mdsal-netconf-connector/src/test/resources/messages/mapping/discardChanges.xml [new file with mode: 0644]
opendaylight/netconf/mdsal-netconf-connector/src/test/resources/messages/mapping/editConfig_create.xml [new file with mode: 0644]
opendaylight/netconf/mdsal-netconf-connector/src/test/resources/messages/mapping/editConfig_delete-root.xml [new file with mode: 0644]
opendaylight/netconf/mdsal-netconf-connector/src/test/resources/messages/mapping/editConfig_merge_missing_default-operation_1.xml [new file with mode: 0644]
opendaylight/netconf/mdsal-netconf-connector/src/test/resources/messages/mapping/editConfig_merge_missing_default-operation_2.xml [new file with mode: 0644]
opendaylight/netconf/mdsal-netconf-connector/src/test/resources/messages/mapping/editConfig_merge_missing_default-operation_control.xml [new file with mode: 0644]
opendaylight/netconf/mdsal-netconf-connector/src/test/resources/messages/mapping/editConfig_merge_multiple_1.xml [new file with mode: 0644]
opendaylight/netconf/mdsal-netconf-connector/src/test/resources/messages/mapping/editConfig_merge_multiple_2.xml [new file with mode: 0644]
opendaylight/netconf/mdsal-netconf-connector/src/test/resources/messages/mapping/editConfig_merge_multiple_3.xml [new file with mode: 0644]
opendaylight/netconf/mdsal-netconf-connector/src/test/resources/messages/mapping/editConfig_merge_multiple_4_replace.xml [new file with mode: 0644]
opendaylight/netconf/mdsal-netconf-connector/src/test/resources/messages/mapping/editConfig_merge_multiple_after_more_complex_merge.xml [new file with mode: 0644]
opendaylight/netconf/mdsal-netconf-connector/src/test/resources/messages/mapping/editConfig_merge_multiple_after_more_complex_merge_2.xml [new file with mode: 0644]
opendaylight/netconf/mdsal-netconf-connector/src/test/resources/messages/mapping/editConfig_merge_multiple_after_replace.xml [new file with mode: 0644]
opendaylight/netconf/mdsal-netconf-connector/src/test/resources/messages/mapping/editConfig_merge_multiple_control_1.xml [new file with mode: 0644]
opendaylight/netconf/mdsal-netconf-connector/src/test/resources/messages/mapping/editConfig_merge_multiple_control_2.xml [new file with mode: 0644]
opendaylight/netconf/mdsal-netconf-connector/src/test/resources/messages/mapping/editConfig_merge_n1.xml [new file with mode: 0644]
opendaylight/netconf/mdsal-netconf-connector/src/test/resources/messages/mapping/editConfig_merge_n1_control.xml [new file with mode: 0644]
opendaylight/netconf/mdsal-netconf-connector/src/test/resources/messages/mapping/editConfig_merge_single_1.xml [new file with mode: 0644]
opendaylight/netconf/mdsal-netconf-connector/src/test/resources/messages/mapping/editConfig_remove-root.xml [new file with mode: 0644]
opendaylight/netconf/mdsal-netconf-connector/src/test/resources/messages/mapping/editConfig_replace_default.xml [new file with mode: 0644]
opendaylight/netconf/mdsal-netconf-connector/src/test/resources/messages/mapping/editConfig_replace_default_control.xml [new file with mode: 0644]
opendaylight/netconf/mdsal-netconf-connector/src/test/resources/messages/mapping/editConfig_running.xml [new file with mode: 0644]
opendaylight/netconf/mdsal-netconf-connector/src/test/resources/messages/mapping/get.xml [new file with mode: 0644]
opendaylight/netconf/mdsal-netconf-connector/src/test/resources/messages/mapping/getConfig.xml [new file with mode: 0644]
opendaylight/netconf/mdsal-netconf-connector/src/test/resources/messages/mapping/getConfig_candidate.xml [new file with mode: 0644]
opendaylight/netconf/mdsal-netconf-connector/src/test/resources/messages/mapping/lock.xml [new file with mode: 0644]
opendaylight/netconf/mdsal-netconf-connector/src/test/resources/messages/mapping/lock_candidate.xml [new file with mode: 0644]
opendaylight/netconf/mdsal-netconf-connector/src/test/resources/messages/mapping/rpc-reply_ok.xml [new file with mode: 0644]
opendaylight/netconf/mdsal-netconf-connector/src/test/resources/messages/mapping/unlock.xml [new file with mode: 0644]
opendaylight/netconf/mdsal-netconf-connector/src/test/resources/messages/mapping/unlock_candidate.xml [new file with mode: 0644]
opendaylight/netconf/mdsal-netconf-connector/src/test/resources/yang/mdsal-netconf-mapping-test.yang [new file with mode: 0644]
opendaylight/netconf/netconf-it/pom.xml

index 51182deb1dcd6586ee1d421d477e3072363d4a83..3029ef7e399a4db99c6ed2ad18a7e7701cb2f8ee 100644 (file)
@@ -11,6 +11,8 @@ package org.opendaylight.controller.cluster.datastore;
 import akka.actor.ActorSystem;
 import com.google.common.annotations.VisibleForTesting;
 import com.google.common.base.Preconditions;
+import java.util.concurrent.CountDownLatch;
+import java.util.concurrent.TimeUnit;
 import org.opendaylight.controller.cluster.datastore.identifiers.ShardManagerIdentifier;
 import org.opendaylight.controller.cluster.datastore.jmx.mbeans.DatastoreConfigurationMXBeanImpl;
 import org.opendaylight.controller.cluster.datastore.shardstrategy.ShardStrategyFactory;
@@ -38,14 +40,22 @@ public class DistributedDataStore implements DOMStore, SchemaContextListener,
         DatastoreContextConfigAdminOverlay.Listener, AutoCloseable {
 
     private static final Logger LOG = LoggerFactory.getLogger(DistributedDataStore.class);
-    public static final int REGISTER_DATA_CHANGE_LISTENER_TIMEOUT_FACTOR = 24; // 24 times the usual operation timeout
+    private static final String UNKNOWN_TYPE = "unknown";
+
+    private static final long READY_WAIT_FACTOR = 3;
 
     private final ActorContext actorContext;
+    private final long waitTillReadyTimeInMillis;
+
 
     private AutoCloseable closeable;
 
     private DatastoreConfigurationMXBeanImpl datastoreConfigMXBean;
 
+    private CountDownLatch waitTillReadyCountDownLatch = new CountDownLatch(1);
+
+    private final String type;
+
     public DistributedDataStore(ActorSystem actorSystem, ClusterWrapper cluster,
             Configuration configuration, DatastoreContext datastoreContext) {
         Preconditions.checkNotNull(actorSystem, "actorSystem should not be null");
@@ -53,7 +63,7 @@ public class DistributedDataStore implements DOMStore, SchemaContextListener,
         Preconditions.checkNotNull(configuration, "configuration should not be null");
         Preconditions.checkNotNull(datastoreContext, "datastoreContext should not be null");
 
-        String type = datastoreContext.getDataStoreType();
+        this.type = datastoreContext.getDataStoreType();
 
         String shardManagerId = ShardManagerIdentifier.builder().type(type).build().toString();
 
@@ -63,10 +73,14 @@ public class DistributedDataStore implements DOMStore, SchemaContextListener,
                 new Dispatchers(actorSystem.dispatchers()).getDispatcherPath(Dispatchers.DispatcherType.Shard);
 
         actorContext = new ActorContext(actorSystem, actorSystem.actorOf(
-                ShardManager.props(cluster, configuration, datastoreContext)
+                ShardManager.props(cluster, configuration, datastoreContext, waitTillReadyCountDownLatch)
                         .withDispatcher(shardDispatcher).withMailbox(ActorContext.MAILBOX), shardManagerId ),
                 cluster, configuration, datastoreContext);
 
+        this.waitTillReadyTimeInMillis =
+                actorContext.getDatastoreContext().getShardLeaderElectionTimeout().duration().toMillis() * READY_WAIT_FACTOR;
+
+
         datastoreConfigMXBean = new DatastoreConfigurationMXBeanImpl(datastoreContext.getDataStoreMXBeanType());
         datastoreConfigMXBean.setContext(datastoreContext);
         datastoreConfigMXBean.registerMBean();
@@ -74,6 +88,10 @@ public class DistributedDataStore implements DOMStore, SchemaContextListener,
 
     public DistributedDataStore(ActorContext actorContext) {
         this.actorContext = Preconditions.checkNotNull(actorContext, "actorContext should not be null");
+        this.type = UNKNOWN_TYPE;
+        this.waitTillReadyTimeInMillis =
+                actorContext.getDatastoreContext().getShardLeaderElectionTimeout().duration().toMillis() * READY_WAIT_FACTOR;
+
     }
 
     public void setCloseable(AutoCloseable closeable) {
@@ -155,4 +173,21 @@ public class DistributedDataStore implements DOMStore, SchemaContextListener,
     ActorContext getActorContext() {
         return actorContext;
     }
+
+    public void waitTillReady(){
+        LOG.info("Beginning to wait for data store to become ready : {}", type);
+
+        try {
+            waitTillReadyCountDownLatch.await(waitTillReadyTimeInMillis, TimeUnit.MILLISECONDS);
+
+            LOG.debug("Data store {} is now ready", type);
+        } catch (InterruptedException e) {
+            LOG.error("Interrupted when trying to wait for shards to become leader in a reasonable amount of time - giving up");
+        }
+    }
+
+    @VisibleForTesting
+    public CountDownLatch getWaitTillReadyCountDownLatch() {
+        return waitTillReadyCountDownLatch;
+    }
 }
index ee9f4f3ad5276e5586da6135902835f0896f2b6f..8199e33294874f729dcc886fa482c9937eb73f1e 100644 (file)
@@ -40,6 +40,7 @@ public class DistributedDataStoreFactory {
         schemaService.registerSchemaContextListener(dataStore);
 
         dataStore.setCloseable(overlay);
+        dataStore.waitTillReady();
         return dataStore;
     }
 
index c509580ac657f920564cc2c4db1a43f91fa7d325..a5abd2fc69059f4af377ab85ac161372de15cbed 100644 (file)
@@ -63,6 +63,7 @@ import org.opendaylight.controller.cluster.datastore.modification.MutableComposi
 import org.opendaylight.controller.cluster.datastore.utils.Dispatchers;
 import org.opendaylight.controller.cluster.datastore.utils.MessageTracker;
 import org.opendaylight.controller.cluster.datastore.utils.SerializationUtils;
+import org.opendaylight.controller.cluster.notifications.RegisterRoleChangeListener;
 import org.opendaylight.controller.cluster.notifications.RoleChangeNotifier;
 import org.opendaylight.controller.cluster.raft.RaftActor;
 import org.opendaylight.controller.cluster.raft.ReplicatedLogEntry;
@@ -215,7 +216,7 @@ public class Shard extends RaftActor {
     private Optional<ActorRef> createRoleChangeNotifier(String shardId) {
         ActorRef shardRoleChangeNotifier = this.getContext().actorOf(
             RoleChangeNotifier.getProps(shardId), shardId + "-notifier");
-        return Optional.<ActorRef>of(shardRoleChangeNotifier);
+        return Optional.of(shardRoleChangeNotifier);
     }
 
     @Override
@@ -288,6 +289,8 @@ public class Shard extends RaftActor {
                 handleTransactionCommitTimeoutCheck();
             } else if(message instanceof DatastoreContext) {
                 onDatastoreContext((DatastoreContext)message);
+            } else if(message instanceof RegisterRoleChangeListener){
+                roleChangeNotifier.get().forward(message, context());
             } else {
                 super.onReceiveCommand(message);
             }
index 775cae35e22843cafd223bce22eb5232a230868f..d836a347c514b434db26a32cbb172d1671e99253 100644 (file)
@@ -22,6 +22,7 @@ import akka.persistence.RecoveryCompleted;
 import akka.persistence.RecoveryFailure;
 import com.google.common.annotations.VisibleForTesting;
 import com.google.common.base.Preconditions;
+import com.google.common.base.Strings;
 import com.google.common.base.Supplier;
 import com.google.common.collect.ImmutableSet;
 import com.google.common.collect.Lists;
@@ -34,6 +35,7 @@ import java.util.HashSet;
 import java.util.List;
 import java.util.Map;
 import java.util.Set;
+import java.util.concurrent.CountDownLatch;
 import org.opendaylight.controller.cluster.DataPersistenceProvider;
 import org.opendaylight.controller.cluster.common.actor.AbstractUntypedPersistentActorWithMetering;
 import org.opendaylight.controller.cluster.datastore.identifiers.ShardIdentifier;
@@ -50,6 +52,9 @@ import org.opendaylight.controller.cluster.datastore.messages.PrimaryFound;
 import org.opendaylight.controller.cluster.datastore.messages.PrimaryNotFound;
 import org.opendaylight.controller.cluster.datastore.messages.UpdateSchemaContext;
 import org.opendaylight.controller.cluster.datastore.utils.Dispatchers;
+import org.opendaylight.controller.cluster.notifications.RegisterRoleChangeListener;
+import org.opendaylight.controller.cluster.notifications.RoleChangeNotification;
+import org.opendaylight.controller.cluster.raft.RaftState;
 import org.opendaylight.yangtools.yang.model.api.ModuleIdentifier;
 import org.opendaylight.yangtools.yang.model.api.SchemaContext;
 import org.slf4j.Logger;
@@ -97,10 +102,12 @@ public class ShardManager extends AbstractUntypedPersistentActorWithMetering {
 
     private final DataPersistenceProvider dataPersistenceProvider;
 
+    private final CountDownLatch waitTillReadyCountdownLatch;
+
     /**
      */
     protected ShardManager(ClusterWrapper cluster, Configuration configuration,
-            DatastoreContext datastoreContext) {
+            DatastoreContext datastoreContext, CountDownLatch waitTillReadyCountdownLatch) {
 
         this.cluster = Preconditions.checkNotNull(cluster, "cluster should not be null");
         this.configuration = Preconditions.checkNotNull(configuration, "configuration should not be null");
@@ -109,6 +116,7 @@ public class ShardManager extends AbstractUntypedPersistentActorWithMetering {
         this.type = datastoreContext.getDataStoreType();
         this.shardDispatcherPath =
                 new Dispatchers(context().system().dispatchers()).getDispatcherPath(Dispatchers.DispatcherType.Shard);
+        this.waitTillReadyCountdownLatch = waitTillReadyCountdownLatch;
 
         // Subscribe this actor to cluster member events
         cluster.subscribeToMemberEvents(getSelf());
@@ -123,12 +131,14 @@ public class ShardManager extends AbstractUntypedPersistentActorWithMetering {
     public static Props props(
         final ClusterWrapper cluster,
         final Configuration configuration,
-        final DatastoreContext datastoreContext) {
+        final DatastoreContext datastoreContext,
+        final CountDownLatch waitTillReadyCountdownLatch) {
 
         Preconditions.checkNotNull(cluster, "cluster should not be null");
         Preconditions.checkNotNull(configuration, "configuration should not be null");
+        Preconditions.checkNotNull(waitTillReadyCountdownLatch, "waitTillReadyCountdownLatch should not be null");
 
-        return Props.create(new ShardManagerCreator(cluster, configuration, datastoreContext));
+        return Props.create(new ShardManagerCreator(cluster, configuration, datastoreContext, waitTillReadyCountdownLatch));
     }
 
     @Override
@@ -156,12 +166,54 @@ public class ShardManager extends AbstractUntypedPersistentActorWithMetering {
             ignoreMessage(message);
         } else if(message instanceof DatastoreContext) {
             onDatastoreContext((DatastoreContext)message);
+        } else if(message instanceof RoleChangeNotification){
+            onRoleChangeNotification((RoleChangeNotification) message);
         } else{
             unknownMessage(message);
         }
 
     }
 
+    private void onRoleChangeNotification(RoleChangeNotification message) {
+        RoleChangeNotification roleChanged = message;
+        LOG.info("Received role changed for {} from {} to {}", roleChanged.getMemberId(),
+                roleChanged.getOldRole(), roleChanged.getNewRole());
+
+        ShardInformation shardInformation = findShardInformation(roleChanged.getMemberId());
+        if(shardInformation != null) {
+            shardInformation.setRole(roleChanged.getNewRole());
+
+            if (isReady()) {
+                LOG.info("All Shards are ready - data store {} is ready, available count is {}", type,
+                        waitTillReadyCountdownLatch.getCount());
+
+                waitTillReadyCountdownLatch.countDown();
+            }
+        }
+    }
+
+
+    private ShardInformation findShardInformation(String memberId) {
+        for(ShardInformation info : localShards.values()){
+            if(info.getShardId().toString().equals(memberId)){
+                return info;
+            }
+        }
+
+        return null;
+    }
+
+    private boolean isReady() {
+        boolean isReady = true;
+        for (ShardInformation info : localShards.values()) {
+            if(RaftState.Candidate.name().equals(info.getRole()) || Strings.isNullOrEmpty(info.getRole())){
+                isReady = false;
+                break;
+            }
+        }
+        return isReady;
+    }
+
     private void onActorInitialized(Object message) {
         final ActorRef sender = getSender();
 
@@ -305,11 +357,12 @@ public class ShardManager extends AbstractUntypedPersistentActorWithMetering {
                     for (ShardInformation info : localShards.values()) {
                         if (info.getActor() == null) {
                             info.setActor(getContext().actorOf(Shard.props(info.getShardId(),
-                                            info.getPeerAddresses(), datastoreContext, schemaContext)
+                                    info.getPeerAddresses(), datastoreContext, schemaContext)
                                             .withDispatcher(shardDispatcherPath), info.getShardId().toString()));
                         } else {
                             info.getActor().tell(message, getSelf());
                         }
+                        info.getActor().tell(new RegisterRoleChangeListener(), self());
                     }
                 }
 
@@ -477,6 +530,7 @@ public class ShardManager extends AbstractUntypedPersistentActorWithMetering {
         private boolean actorInitialized = false;
 
         private final List<Runnable> runnablesOnInitialized = Lists.newArrayList();
+        private String role ;
 
         private ShardInformation(String shardName, ShardIdentifier shardId,
                 Map<ShardIdentifier, String> peerAddresses) {
@@ -544,6 +598,15 @@ public class ShardManager extends AbstractUntypedPersistentActorWithMetering {
         void addRunnableOnInitialized(Runnable runnable) {
             runnablesOnInitialized.add(runnable);
         }
+
+        public void setRole(String newRole) {
+            this.role = newRole;
+        }
+
+        public String getRole(){
+            return this.role;
+        }
+
     }
 
     private static class ShardManagerCreator implements Creator<ShardManager> {
@@ -552,17 +615,19 @@ public class ShardManager extends AbstractUntypedPersistentActorWithMetering {
         final ClusterWrapper cluster;
         final Configuration configuration;
         final DatastoreContext datastoreContext;
+        private final CountDownLatch waitTillReadyCountdownLatch;
 
         ShardManagerCreator(ClusterWrapper cluster,
-                Configuration configuration, DatastoreContext datastoreContext) {
+                            Configuration configuration, DatastoreContext datastoreContext, CountDownLatch waitTillReadyCountdownLatch) {
             this.cluster = cluster;
             this.configuration = configuration;
             this.datastoreContext = datastoreContext;
+            this.waitTillReadyCountdownLatch = waitTillReadyCountdownLatch;
         }
 
         @Override
         public ShardManager create() throws Exception {
-            return new ShardManager(cluster, configuration, datastoreContext);
+            return new ShardManager(cluster, configuration, datastoreContext, waitTillReadyCountdownLatch);
         }
     }
 
index 2757d2f5f68eef469361bd940866d2705b5494cc..74e61c189f828bb9228152ada1dedae2b16c53e3 100644 (file)
@@ -217,7 +217,8 @@ public class MessageTracker {
         boolean done = true;
 
         public void reset(){
-            Preconditions.checkState(done);
+            Preconditions.checkState(done,
+                    String.format("Trying to reset a context that is not done (%s). currentMessage = %s", done, currentMessage));
             done = false;
             stopwatch.reset().start();
         }
index 4ec035ee3b52308e6390d9a2e4dece6703438318..3034004bb0ad2209f43602749206a36f187c67ba 100644 (file)
@@ -1,8 +1,13 @@
 package org.opendaylight.controller.cluster.datastore;
 
+import static org.junit.Assert.assertTrue;
 import static org.mockito.Mockito.doReturn;
 import static org.mockito.Mockito.times;
 import static org.mockito.Mockito.verify;
+import akka.util.Timeout;
+import com.google.common.util.concurrent.Uninterruptibles;
+import java.util.concurrent.Executors;
+import java.util.concurrent.TimeUnit;
 import org.junit.Before;
 import org.junit.Test;
 import org.mockito.Mock;
@@ -10,6 +15,7 @@ import org.mockito.MockitoAnnotations;
 import org.opendaylight.controller.cluster.datastore.utils.ActorContext;
 import org.opendaylight.controller.md.cluster.datastore.model.TestModel;
 import org.opendaylight.yangtools.yang.model.api.SchemaContext;
+import scala.concurrent.duration.FiniteDuration;
 
 public class DistributedDataStoreTest extends AbstractActorTest {
 
@@ -18,6 +24,12 @@ public class DistributedDataStoreTest extends AbstractActorTest {
     @Mock
     private ActorContext actorContext;
 
+    @Mock
+    private DatastoreContext datastoreContext;
+
+    @Mock
+    private Timeout shardElectionTimeout;
+
     @Before
     public void setUp() throws Exception {
         MockitoAnnotations.initMocks(this);
@@ -58,4 +70,45 @@ public class DistributedDataStoreTest extends AbstractActorTest {
         verify(actorContext, times(0)).acquireTxCreationPermit();
     }
 
+    @Test
+    public void testWaitTillReadyBlocking(){
+        doReturn(datastoreContext).when(actorContext).getDatastoreContext();
+        doReturn(shardElectionTimeout).when(datastoreContext).getShardLeaderElectionTimeout();
+        doReturn(FiniteDuration.apply(50, TimeUnit.MILLISECONDS)).when(shardElectionTimeout).duration();
+        DistributedDataStore distributedDataStore = new DistributedDataStore(actorContext);
+
+        long start = System.currentTimeMillis();
+
+        distributedDataStore.waitTillReady();
+
+        long end = System.currentTimeMillis();
+
+        assertTrue("Expected to be blocked for 50 millis", (end-start) >= 50);
+    }
+
+    @Test
+    public void testWaitTillReadyCountDown(){
+        final DistributedDataStore distributedDataStore = new DistributedDataStore(actorContext);
+        doReturn(datastoreContext).when(actorContext).getDatastoreContext();
+        doReturn(shardElectionTimeout).when(datastoreContext).getShardLeaderElectionTimeout();
+        doReturn(FiniteDuration.apply(5000, TimeUnit.MILLISECONDS)).when(shardElectionTimeout).duration();
+
+        Executors.newSingleThreadExecutor().submit(new Runnable() {
+            @Override
+            public void run() {
+                Uninterruptibles.sleepUninterruptibly(500, TimeUnit.MILLISECONDS);
+                distributedDataStore.getWaitTillReadyCountDownLatch().countDown();
+            }
+        });
+
+        long start = System.currentTimeMillis();
+
+        distributedDataStore.waitTillReady();
+
+        long end = System.currentTimeMillis();
+
+        assertTrue("Expected to be released in 500 millis", (end-start) < 5000);
+
+    }
+
 }
\ No newline at end of file
index 596761ddc8fa9e9d25b4c797c2f814f5a1c63a09..f0cdacc9ef2bc0e08ccb0348a8d990279de8f57a 100644 (file)
@@ -4,6 +4,9 @@ import static org.junit.Assert.assertEquals;
 import static org.junit.Assert.assertFalse;
 import static org.junit.Assert.assertTrue;
 import static org.mockito.Mockito.mock;
+import static org.mockito.Mockito.never;
+import static org.mockito.Mockito.times;
+import static org.mockito.Mockito.verify;
 import static org.mockito.Mockito.when;
 import akka.actor.ActorRef;
 import akka.actor.Props;
@@ -26,6 +29,8 @@ import java.util.concurrent.TimeUnit;
 import org.junit.After;
 import org.junit.Before;
 import org.junit.Test;
+import org.mockito.Mock;
+import org.mockito.MockitoAnnotations;
 import org.opendaylight.controller.cluster.DataPersistenceProvider;
 import org.opendaylight.controller.cluster.datastore.identifiers.ShardIdentifier;
 import org.opendaylight.controller.cluster.datastore.messages.ActorInitialized;
@@ -41,6 +46,8 @@ import org.opendaylight.controller.cluster.datastore.utils.DoNothingActor;
 import org.opendaylight.controller.cluster.datastore.utils.InMemoryJournal;
 import org.opendaylight.controller.cluster.datastore.utils.MockClusterWrapper;
 import org.opendaylight.controller.cluster.datastore.utils.MockConfiguration;
+import org.opendaylight.controller.cluster.notifications.RoleChangeNotification;
+import org.opendaylight.controller.cluster.raft.RaftState;
 import org.opendaylight.controller.md.cluster.datastore.model.TestModel;
 import org.opendaylight.yangtools.yang.model.api.ModuleIdentifier;
 import org.opendaylight.yangtools.yang.model.api.SchemaContext;
@@ -53,10 +60,15 @@ public class ShardManagerTest extends AbstractActorTest {
     private final String shardMrgIDSuffix = "config" + ID_COUNTER++;
     private final String shardMgrID = "shard-manager-" + shardMrgIDSuffix;
 
+    @Mock
+    private static CountDownLatch ready;
+
     private static ActorRef mockShardActor;
 
     @Before
     public void setUp() {
+        MockitoAnnotations.initMocks(this);
+
         InMemoryJournal.clear();
 
         if(mockShardActor == null) {
@@ -71,10 +83,10 @@ public class ShardManagerTest extends AbstractActorTest {
     }
 
     private Props newShardMgrProps() {
-
         DatastoreContext.Builder builder = DatastoreContext.newBuilder();
         builder.dataStoreType(shardMrgIDSuffix);
-        return ShardManager.props(new MockClusterWrapper(), new MockConfiguration(), builder.build());
+        return ShardManager.props(new MockClusterWrapper(), new MockConfiguration(),
+                builder.build(), ready);
     }
 
     @Test
@@ -351,8 +363,10 @@ public class ShardManagerTest extends AbstractActorTest {
     public void testRecoveryApplicable(){
         new JavaTestKit(getSystem()) {
             {
-                final Props persistentProps = ShardManager.props(new MockClusterWrapper(), new MockConfiguration(),
-                        DatastoreContext.newBuilder().persistent(true).dataStoreType(shardMrgIDSuffix).build());
+                final Props persistentProps = ShardManager.props(
+                        new MockClusterWrapper(),
+                        new MockConfiguration(),
+                        DatastoreContext.newBuilder().persistent(true).build(), ready);
                 final TestActorRef<ShardManager> persistentShardManager =
                         TestActorRef.create(getSystem(), persistentProps);
 
@@ -360,8 +374,10 @@ public class ShardManagerTest extends AbstractActorTest {
 
                 assertTrue("Recovery Applicable", dataPersistenceProvider1.isRecoveryApplicable());
 
-                final Props nonPersistentProps = ShardManager.props(new MockClusterWrapper(), new MockConfiguration(),
-                        DatastoreContext.newBuilder().persistent(false).dataStoreType(shardMrgIDSuffix).build());
+                final Props nonPersistentProps = ShardManager.props(
+                        new MockClusterWrapper(),
+                        new MockConfiguration(),
+                        DatastoreContext.newBuilder().persistent(false).build(), ready);
                 final TestActorRef<ShardManager> nonPersistentShardManager =
                         TestActorRef.create(getSystem(), nonPersistentProps);
 
@@ -382,8 +398,7 @@ public class ShardManagerTest extends AbstractActorTest {
             private static final long serialVersionUID = 1L;
             @Override
             public ShardManager create() throws Exception {
-                return new ShardManager(new MockClusterWrapper(), new MockConfiguration(),
-                        DatastoreContext.newBuilder().dataStoreType(shardMrgIDSuffix).build()) {
+                return new ShardManager(new MockClusterWrapper(), new MockConfiguration(), DatastoreContext.newBuilder().build(), ready) {
                     @Override
                     protected DataPersistenceProvider createDataPersistenceProvider(boolean persistent) {
                         DataPersistenceProviderMonitor dataPersistenceProviderMonitor
@@ -417,6 +432,42 @@ public class ShardManagerTest extends AbstractActorTest {
         }};
     }
 
+    @Test
+    public void testRoleChangeNotificationReleaseReady() throws Exception {
+        new JavaTestKit(getSystem()) {
+            {
+                final Props persistentProps = ShardManager.props(
+                        new MockClusterWrapper(),
+                        new MockConfiguration(),
+                        DatastoreContext.newBuilder().persistent(true).build(), ready);
+                final TestActorRef<ShardManager> shardManager =
+                        TestActorRef.create(getSystem(), persistentProps);
+
+                shardManager.underlyingActor().onReceiveCommand(new RoleChangeNotification("member-1-shard-default-unknown", RaftState.Candidate.name(), RaftState.Leader.name()));
+
+                verify(ready, times(1)).countDown();
+
+            }};
+    }
+
+    @Test
+    public void testRoleChangeNotificationDoNothingForUnknownShard() throws Exception {
+        new JavaTestKit(getSystem()) {
+            {
+                final Props persistentProps = ShardManager.props(
+                        new MockClusterWrapper(),
+                        new MockConfiguration(),
+                        DatastoreContext.newBuilder().persistent(true).build(), ready);
+                final TestActorRef<ShardManager> shardManager =
+                        TestActorRef.create(getSystem(), persistentProps);
+
+                shardManager.underlyingActor().onReceiveCommand(new RoleChangeNotification("unknown", RaftState.Candidate.name(), RaftState.Leader.name()));
+
+                verify(ready, never()).countDown();
+
+            }};
+    }
+
 
 
     private static class TestShardManager extends ShardManager {
@@ -424,7 +475,7 @@ public class ShardManagerTest extends AbstractActorTest {
 
         TestShardManager(String shardMrgIDSuffix) {
             super(new MockClusterWrapper(), new MockConfiguration(),
-                    DatastoreContext.newBuilder().dataStoreType(shardMrgIDSuffix).build());
+                    DatastoreContext.newBuilder().dataStoreType(shardMrgIDSuffix).build(), ready);
         }
 
         @Override
index 1ebd1b91dd474bf14ce0c5d9c32a9eb8cce64d84..7dfbd668b811231b4b32edd76a04987c671aba99 100644 (file)
@@ -32,6 +32,7 @@ import com.google.common.util.concurrent.Uninterruptibles;
 import java.io.IOException;
 import java.util.Collections;
 import java.util.HashSet;
+import java.util.List;
 import java.util.Map;
 import java.util.Set;
 import java.util.concurrent.CountDownLatch;
@@ -69,8 +70,11 @@ import org.opendaylight.controller.cluster.datastore.modification.WriteModificat
 import org.opendaylight.controller.cluster.datastore.node.NormalizedNodeToNodeCodec;
 import org.opendaylight.controller.cluster.datastore.utils.InMemoryJournal;
 import org.opendaylight.controller.cluster.datastore.utils.InMemorySnapshotStore;
+import org.opendaylight.controller.cluster.datastore.utils.MessageCollectorActor;
 import org.opendaylight.controller.cluster.datastore.utils.MockDataChangeListener;
 import org.opendaylight.controller.cluster.datastore.utils.SerializationUtils;
+import org.opendaylight.controller.cluster.notifications.RegisterRoleChangeListener;
+import org.opendaylight.controller.cluster.notifications.RegisterRoleChangeListenerReply;
 import org.opendaylight.controller.cluster.raft.ReplicatedLogEntry;
 import org.opendaylight.controller.cluster.raft.ReplicatedLogImplEntry;
 import org.opendaylight.controller.cluster.raft.Snapshot;
@@ -1591,6 +1595,33 @@ public class ShardTest extends AbstractActorTest {
         }};
     }
 
+    @Test
+    public void testRegisterRoleChangeListener() throws Exception {
+        new ShardTestKit(getSystem()) {
+            {
+                final TestActorRef<Shard> shard = TestActorRef.create(getSystem(),
+                        newShardProps().withDispatcher(Dispatchers.DefaultDispatcherId()),
+                        "testRegisterRoleChangeListener");
+
+                waitUntilLeader(shard);
+
+                TestActorRef<MessageCollectorActor> listener =
+                        TestActorRef.create(getSystem(), Props.create(MessageCollectorActor.class));
+
+                shard.tell(new RegisterRoleChangeListener(), listener);
+
+                // TODO: MessageCollectorActor exists as a test util in both the akka-raft and distributed-datastore
+                // projects. Need to move it to commons as a regular utility and then we can get rid of this arbitrary
+                // sleep.
+                Uninterruptibles.sleepUninterruptibly(100, TimeUnit.MILLISECONDS);
+
+                List<Object> allMatching = MessageCollectorActor.getAllMatching(listener, RegisterRoleChangeListenerReply.class);
+
+                assertEquals(1, allMatching.size());
+            }};
+    }
+
+
     private NormalizedNode<?, ?> readStore(final InMemoryDOMDataStore store) throws ReadFailedException {
         DOMStoreReadTransaction transaction = store.newReadOnlyTransaction();
         CheckedFuture<Optional<NormalizedNode<?, ?>>, ReadFailedException> read =
index 8b3b196049cdbefb289a5abf291fbd75056c79e7..aace7c3cb29ad309ed015727983cb4414def84de 100644 (file)
     <dependency>
       <groupId>org.opendaylight.yangtools</groupId>
       <artifactId>yang-data-impl</artifactId>
+      <exclusions>
+          <exclusion>
+              <groupId>org.opendaylight.yangtools</groupId>
+              <artifactId>object-cache-api</artifactId>
+          </exclusion>
+      </exclusions>
+
     </dependency>
-    <dependency>
+      <dependency>
+          <groupId>org.opendaylight.yangtools</groupId>
+          <artifactId>object-cache-noop</artifactId>
+      </dependency>
+      <dependency>
       <groupId>org.opendaylight.controller</groupId>
       <artifactId>sal-core-api</artifactId>
     </dependency>
       <groupId>org.opendaylight.controller</groupId>
       <artifactId>config-util</artifactId>
     </dependency>
+      <dependency>
+          <groupId>org.opendaylight.controller</groupId>
+          <artifactId>config-api</artifactId>
+      </dependency>
       <dependency>
           <groupId>org.opendaylight.yangtools</groupId>
           <artifactId>yang-data-operations</artifactId>
       </dependency>
+      <dependency>
+          <groupId>org.opendaylight.controller</groupId>
+          <artifactId>sal-broker-impl</artifactId>
+          <scope>test</scope>
+      </dependency>
+      <dependency>
+          <groupId>org.opendaylight.controller</groupId>
+          <artifactId>sal-distributed-datastore</artifactId>
+          <scope>test</scope>
+      </dependency>
+      <dependency>
+          <groupId>org.opendaylight.controller</groupId>
+          <artifactId>netconf-util</artifactId>
+          <type>test-jar</type>
+          <scope>test</scope>
+      </dependency>
+      <dependency>
+          <groupId>xmlunit</groupId>
+          <artifactId>xmlunit</artifactId>
+          <scope>test</scope>
+      </dependency>
 
   </dependencies>
 
diff --git a/opendaylight/netconf/mdsal-netconf-connector/src/test/java/org/opendaylight/controller/netconf/mdsal/connector/ops/NetconfMDSalMappingTest.java b/opendaylight/netconf/mdsal-netconf-connector/src/test/java/org/opendaylight/controller/netconf/mdsal/connector/ops/NetconfMDSalMappingTest.java
new file mode 100644 (file)
index 0000000..ca9948a
--- /dev/null
@@ -0,0 +1,443 @@
+/*
+ * Copyright (c) 2015 Cisco Systems, Inc. and others.  All rights reserved.
+ *
+ * This program and the accompanying materials are made available under the
+ * terms of the Eclipse Public License v1.0 which accompanies this distribution,
+ * and is available at http://www.eclipse.org/legal/epl-v10.html
+ */
+
+package org.opendaylight.controller.netconf.mdsal.connector.ops;
+
+import static org.junit.Assert.assertEquals;
+import static org.junit.Assert.assertFalse;
+import static org.junit.Assert.assertTrue;
+import static org.junit.Assert.fail;
+
+import com.google.common.base.Preconditions;
+import com.google.common.io.ByteSource;
+import java.io.IOException;
+import java.io.InputStream;
+import java.util.ArrayList;
+import java.util.Arrays;
+import java.util.Collection;
+import java.util.EnumMap;
+import java.util.List;
+import java.util.concurrent.ExecutorService;
+import javax.xml.parsers.ParserConfigurationException;
+import org.custommonkey.xmlunit.DetailedDiff;
+import org.custommonkey.xmlunit.Diff;
+import org.custommonkey.xmlunit.XMLUnit;
+import org.custommonkey.xmlunit.examples.RecursiveElementNameAndTextQualifier;
+import org.junit.Before;
+import org.junit.Test;
+import org.opendaylight.controller.cluster.datastore.ConcurrentDOMDataBroker;
+import org.opendaylight.controller.md.sal.common.api.data.LogicalDatastoreType;
+import org.opendaylight.controller.md.sal.dom.store.impl.InMemoryDOMDataStoreFactory;
+import org.opendaylight.controller.netconf.api.NetconfDocumentedException;
+import org.opendaylight.controller.netconf.api.NetconfDocumentedException.ErrorSeverity;
+import org.opendaylight.controller.netconf.api.NetconfDocumentedException.ErrorTag;
+import org.opendaylight.controller.netconf.api.NetconfDocumentedException.ErrorType;
+import org.opendaylight.controller.netconf.mapping.api.HandlingPriority;
+import org.opendaylight.controller.netconf.mapping.api.NetconfOperation;
+import org.opendaylight.controller.netconf.mapping.api.NetconfOperationChainedExecution;
+import org.opendaylight.controller.netconf.mdsal.connector.CurrentSchemaContext;
+import org.opendaylight.controller.netconf.mdsal.connector.TransactionProvider;
+import org.opendaylight.controller.netconf.mdsal.connector.ops.get.Get;
+import org.opendaylight.controller.netconf.mdsal.connector.ops.get.GetConfig;
+import org.opendaylight.controller.netconf.util.test.XmlFileLoader;
+import org.opendaylight.controller.netconf.util.xml.XmlUtil;
+import org.opendaylight.controller.sal.core.api.model.SchemaService;
+import org.opendaylight.controller.sal.core.spi.data.DOMStore;
+import org.opendaylight.yangtools.concepts.ListenerRegistration;
+import org.opendaylight.yangtools.util.concurrent.SpecialExecutors;
+import org.opendaylight.yangtools.yang.model.api.Module;
+import org.opendaylight.yangtools.yang.model.api.SchemaContext;
+import org.opendaylight.yangtools.yang.model.api.SchemaContextListener;
+import org.opendaylight.yangtools.yang.model.parser.api.YangSyntaxErrorException;
+import org.opendaylight.yangtools.yang.parser.builder.impl.BuilderUtils;
+import org.opendaylight.yangtools.yang.parser.impl.YangParserImpl;
+import org.slf4j.Logger;
+import org.slf4j.LoggerFactory;
+import org.w3c.dom.Document;
+import org.w3c.dom.Node;
+import org.w3c.dom.NodeList;
+import org.xml.sax.SAXException;
+
+public class NetconfMDSalMappingTest {
+
+    private static final Logger LOG = LoggerFactory.getLogger(NetconfMDSalMappingTest.class);
+
+    private static final String RPC_REPLY_ELEMENT = "rpc-reply";
+    private static final String DATA_ELEMENT = "data";
+
+    private static Document RPC_REPLY_OK = null;
+
+    static {
+        try {
+            RPC_REPLY_OK = XmlFileLoader.xmlFileToDocument("messages/mapping/rpc-reply_ok.xml");
+        } catch (Exception e) {
+            LOG.debug("unable to load rpc reply ok.", e);
+            RPC_REPLY_OK = XmlUtil.newDocument();
+        }
+    }
+
+    private CurrentSchemaContext currentSchemaContext = null;
+    private SchemaContext schemaContext = null;
+    private String sessionIdForReporting = "netconf-test-session1";
+
+    private TransactionProvider transactionProvider = null;
+
+
+    @Before
+    public void setUp() throws Exception {
+
+        XMLUnit.setIgnoreWhitespace(true);
+        XMLUnit.setIgnoreAttributeOrder(true);
+
+        this.schemaContext = parseSchemas(getYangSchemas());
+        schemaContext.getModules();
+        SchemaService schemaService = createSchemaService();
+
+        final DOMStore operStore = InMemoryDOMDataStoreFactory.create("DOM-OPER", schemaService);
+        final DOMStore configStore = InMemoryDOMDataStoreFactory.create("DOM-CFG", schemaService);
+
+        final EnumMap<LogicalDatastoreType, DOMStore> datastores = new EnumMap<>(LogicalDatastoreType.class);
+        datastores.put(LogicalDatastoreType.CONFIGURATION, configStore);
+        datastores.put(LogicalDatastoreType.OPERATIONAL, operStore);
+
+        ExecutorService listenableFutureExecutor = SpecialExecutors.newBlockingBoundedCachedThreadPool(
+                16, 16, "CommitFutures");
+
+        ConcurrentDOMDataBroker cdb = new ConcurrentDOMDataBroker(datastores, listenableFutureExecutor);
+        this.transactionProvider = new TransactionProvider(cdb, sessionIdForReporting);
+        this.currentSchemaContext = new CurrentSchemaContext(schemaService);
+
+    }
+
+    @Test
+    public void testEmptyDatastore() throws Exception {
+
+        Document response = getConfigRunning();
+        assertEmptyDatastore(response);
+
+        response = getConfigCandidate();
+        assertEmptyDatastore(response);
+
+        response = get();
+        assertEmptyDatastore(response);
+
+    }
+
+    @Test
+    public void testEditRunning() throws Exception {
+
+        try {
+            edit("messages/mapping/editConfig_running.xml");
+            fail("Should have failed - edit config on running datastore is not supported");
+        } catch (NetconfDocumentedException e) {
+            assertTrue(e.getErrorSeverity() == ErrorSeverity.error);
+            assertTrue(e.getErrorTag() == ErrorTag.operation_not_supported);
+            assertTrue(e.getErrorType() == ErrorType.protocol);
+        }
+
+    }
+
+    @Test
+    public void testCandidateTransaction() throws Exception {
+
+        verifyResponse(edit("messages/mapping/editConfig_merge_n1.xml"), RPC_REPLY_OK);
+        verifyResponse(getConfigCandidate(), XmlFileLoader.xmlFileToDocument("messages/mapping/editConfig_merge_n1_control.xml"));
+        assertEmptyDatastore(getConfigRunning());
+
+        verifyResponse(discardChanges(), RPC_REPLY_OK);
+        assertEmptyDatastore(getConfigCandidate());
+
+    }
+
+    @Test
+    public void testEditWithCommit() throws Exception {
+
+        verifyResponse(edit("messages/mapping/editConfig_merge_n1.xml"), RPC_REPLY_OK);
+        verifyResponse(getConfigCandidate(), XmlFileLoader.xmlFileToDocument("messages/mapping/editConfig_merge_n1_control.xml"));
+
+        verifyResponse(commit(), RPC_REPLY_OK);
+        verifyResponse(getConfigRunning(), XmlFileLoader.xmlFileToDocument("messages/mapping/editConfig_merge_n1_control.xml"));
+
+        deleteDatastore();
+
+    }
+
+    @Test
+    public void testMultipleEditsWithMerge() throws Exception {
+
+        verifyResponse(edit("messages/mapping/editConfig_merge_multiple_1.xml"), RPC_REPLY_OK);
+        verifyResponse(getConfigCandidate(), XmlFileLoader.xmlFileToDocument("messages/mapping/editConfig_merge_multiple_control_1.xml"));
+        verifyResponse(edit("messages/mapping/editConfig_merge_single_1.xml"), RPC_REPLY_OK);
+        verifyResponse(getConfigCandidate(), XmlFileLoader.xmlFileToDocument("messages/mapping/editConfig_merge_multiple_control_2.xml"));
+        assertEmptyDatastore(getConfigRunning());
+
+        verifyResponse(commit(), RPC_REPLY_OK);
+        verifyResponse(getConfigRunning(), XmlFileLoader.xmlFileToDocument("messages/mapping/editConfig_merge_multiple_control_2.xml"));
+
+        deleteDatastore();
+
+    }
+
+    @Test
+    public void testMoreComplexEditConfigs() throws Exception {
+
+        verifyResponse(edit("messages/mapping/editConfig_merge_multiple_1.xml"), RPC_REPLY_OK);
+        verifyResponse(edit("messages/mapping/editConfig_merge_single_1.xml"), RPC_REPLY_OK);
+
+        verifyResponse(edit("messages/mapping/editConfig_merge_multiple_2.xml"), RPC_REPLY_OK);
+        verifyResponse(getConfigCandidate(), XmlFileLoader.xmlFileToDocument("messages/mapping/editConfig_merge_multiple_after_more_complex_merge.xml"));
+
+        verifyResponse(edit("messages/mapping/editConfig_merge_multiple_3.xml"), RPC_REPLY_OK);
+        verifyResponse(getConfigCandidate(), XmlFileLoader.xmlFileToDocument("messages/mapping/editConfig_merge_multiple_after_more_complex_merge_2.xml"));
+
+        verifyResponse(edit("messages/mapping/editConfig_merge_multiple_4_replace.xml"), RPC_REPLY_OK);
+        verifyResponse(getConfigCandidate(), XmlFileLoader.xmlFileToDocument("messages/mapping/editConfig_merge_multiple_after_replace.xml"));
+        verifyResponse(commit(), RPC_REPLY_OK);
+
+        verifyResponse(getConfigRunning(), XmlFileLoader.xmlFileToDocument("messages/mapping/editConfig_merge_multiple_after_replace.xml"));
+
+        verifyResponse(edit("messages/mapping/editConfig_replace_default.xml"), RPC_REPLY_OK);
+        verifyResponse(getConfigCandidate(), XmlFileLoader.xmlFileToDocument("messages/mapping/editConfig_replace_default_control.xml"));
+        verifyResponse(commit(), RPC_REPLY_OK);
+
+        verifyResponse(getConfigRunning(), XmlFileLoader.xmlFileToDocument("messages/mapping/editConfig_replace_default_control.xml"));
+
+        deleteDatastore();
+
+    }
+
+    @Test
+    public void testLock() throws Exception {
+
+        verifyResponse(lockCandidate(), RPC_REPLY_OK);
+
+        try {
+            lock();
+            fail("Should have failed - locking of running datastore is not supported");
+        } catch (NetconfDocumentedException e) {
+            assertTrue(e.getErrorSeverity() == ErrorSeverity.error);
+            assertTrue(e.getErrorTag() == ErrorTag.operation_not_supported);
+            assertTrue(e.getErrorType() == ErrorType.application);
+        }
+    }
+
+    @Test
+    public void testUnlock() throws Exception {
+
+        verifyResponse(unlockCandidate(), RPC_REPLY_OK);
+
+        try {
+            unlock();
+            fail("Should have failed - unlocking of running datastore is not supported");
+        } catch (NetconfDocumentedException e) {
+            assertTrue(e.getErrorSeverity() == ErrorSeverity.error);
+            assertTrue(e.getErrorTag() == ErrorTag.operation_not_supported);
+            assertTrue(e.getErrorType() == ErrorType.application);
+        }
+    }
+
+    @Test
+    public void testEditWithCreate() throws Exception {
+
+        verifyResponse(edit("messages/mapping/editConfig_create.xml"), RPC_REPLY_OK);
+        verifyResponse(getConfigCandidate(), XmlFileLoader.xmlFileToDocument("messages/mapping/editConfig_merge_n1_control.xml"));
+
+        try {
+            edit("messages/mapping/editConfig_create.xml");
+            fail("Create should have failed - data already exists");
+        } catch (NetconfDocumentedException e) {
+            assertTrue(e.getErrorSeverity() == ErrorSeverity.error);
+            assertTrue(e.getErrorTag() == ErrorTag.data_exists);
+            assertTrue(e.getErrorType() == ErrorType.protocol);
+        }
+
+        verifyResponse(discardChanges(), RPC_REPLY_OK);
+
+    }
+
+    @Test
+    public void testDeleteNonExisting() throws Exception {
+
+        assertEmptyDatastore(getConfigCandidate());
+        assertEmptyDatastore(getConfigRunning());
+
+        try {
+            edit("messages/mapping/editConfig_delete-root.xml");
+            fail("Delete should have failed - data is missing");
+        } catch (NetconfDocumentedException e) {
+            assertTrue(e.getErrorSeverity() == ErrorSeverity.error);
+            assertTrue(e.getErrorTag() == ErrorTag.data_missing);
+            assertTrue(e.getErrorType() == ErrorType.protocol);
+        }
+
+    }
+
+    @Test
+    public void testEditMissingDefaultOperation() throws Exception {
+
+        verifyResponse(edit("messages/mapping/editConfig_merge_missing_default-operation_1.xml"), RPC_REPLY_OK);
+        verifyResponse(edit("messages/mapping/editConfig_merge_missing_default-operation_2.xml"), RPC_REPLY_OK);
+        verifyResponse(getConfigCandidate(), XmlFileLoader.xmlFileToDocument("messages/mapping/editConfig_merge_missing_default-operation_control.xml"));
+
+        verifyResponse(commit(), RPC_REPLY_OK);
+        verifyResponse(getConfigRunning(), XmlFileLoader.xmlFileToDocument("messages/mapping/editConfig_merge_missing_default-operation_control.xml"));
+
+        deleteDatastore();
+    }
+
+    private void deleteDatastore() throws Exception{
+        verifyResponse(edit("messages/mapping/editConfig_delete-root.xml"), RPC_REPLY_OK);
+        assertEmptyDatastore(getConfigCandidate());
+
+        verifyResponse(commit(), RPC_REPLY_OK);
+        assertEmptyDatastore(getConfigRunning());
+    }
+
+    private void verifyResponse(Document response, Document template) {
+        DetailedDiff dd = new DetailedDiff(new Diff(response, template));
+        dd.overrideElementQualifier(new RecursiveElementNameAndTextQualifier());
+        assertTrue(dd.similar());
+    }
+
+    private void assertEmptyDatastore(Document response) {
+
+        NodeList nodes = response.getChildNodes();
+        assertTrue(nodes.getLength() == 1);
+
+        assertEquals(nodes.item(0).getLocalName(),RPC_REPLY_ELEMENT);
+
+        NodeList replyNodes = nodes.item(0).getChildNodes();
+        assertTrue(replyNodes.getLength() == 1);
+
+        Node dataNode = replyNodes.item(0);
+        assertEquals(dataNode.getLocalName(), DATA_ELEMENT);
+        assertFalse(dataNode.hasChildNodes());
+
+    }
+
+    private Document commit() throws NetconfDocumentedException, ParserConfigurationException, SAXException, IOException {
+        Commit commit = new Commit(sessionIdForReporting, transactionProvider);
+        return executeOperation(commit, "messages/mapping/commit.xml");
+    }
+
+    private Document discardChanges() throws NetconfDocumentedException, ParserConfigurationException, SAXException, IOException {
+        DiscardChanges discardOp = new DiscardChanges(sessionIdForReporting, transactionProvider);
+        return executeOperation(discardOp, "messages/mapping/discardChanges.xml");
+    }
+
+    private Document edit(String resource) throws NetconfDocumentedException, ParserConfigurationException, SAXException, IOException {
+        EditConfig editConfig = new EditConfig(sessionIdForReporting, currentSchemaContext, transactionProvider);
+        return executeOperation(editConfig, resource);
+    }
+
+    private Document get() throws NetconfDocumentedException, ParserConfigurationException, SAXException, IOException {
+        Get get = new Get(sessionIdForReporting, currentSchemaContext, transactionProvider);
+        return executeOperation(get, "messages/mapping/get.xml");
+    }
+
+    private Document getConfigRunning() throws NetconfDocumentedException, ParserConfigurationException, SAXException, IOException {
+        GetConfig getConfig = new GetConfig(sessionIdForReporting, currentSchemaContext, transactionProvider);
+        return executeOperation(getConfig, "messages/mapping/getConfig.xml");
+    }
+
+    private Document getConfigCandidate() throws NetconfDocumentedException, ParserConfigurationException, SAXException, IOException {
+        GetConfig getConfig = new GetConfig(sessionIdForReporting, currentSchemaContext, transactionProvider);
+        return executeOperation(getConfig, "messages/mapping/getConfig_candidate.xml");
+    }
+
+    private Document lock() throws NetconfDocumentedException, ParserConfigurationException, SAXException, IOException {
+        Lock lock = new Lock(sessionIdForReporting);
+        return executeOperation(lock, "messages/mapping/lock.xml");
+    }
+
+    private Document unlock() throws NetconfDocumentedException, ParserConfigurationException, SAXException, IOException {
+        Unlock unlock = new Unlock(sessionIdForReporting);
+        return executeOperation(unlock, "messages/mapping/unlock.xml");
+    }
+
+    private Document lockCandidate() throws NetconfDocumentedException, ParserConfigurationException, SAXException, IOException {
+        Lock lock = new Lock(sessionIdForReporting);
+        return executeOperation(lock, "messages/mapping/lock_candidate.xml");
+    }
+
+    private Document unlockCandidate() throws NetconfDocumentedException, ParserConfigurationException, SAXException, IOException {
+        Unlock unlock = new Unlock(sessionIdForReporting);
+        return executeOperation(unlock, "messages/mapping/unlock_candidate.xml");
+    }
+
+    private Document executeOperation(NetconfOperation op, String filename) throws ParserConfigurationException, SAXException, IOException, NetconfDocumentedException {
+        final Document request = XmlFileLoader.xmlFileToDocument(filename);
+
+        HandlingPriority priority = op.canHandle(request);
+        Preconditions.checkState(priority != HandlingPriority.CANNOT_HANDLE);
+
+        final Document response = op.handle(request, NetconfOperationChainedExecution.EXECUTION_TERMINATION_POINT);
+
+        LOG.debug("Got response {}" , response);
+
+        return response;
+    }
+
+    private Collection<InputStream> getYangSchemas() {
+        final List<String> schemaPaths = Arrays.asList("/META-INF/yang/config.yang", "/yang/mdsal-netconf-mapping-test.yang");
+        final List<InputStream> schemas = new ArrayList<>();
+
+        for (String schemaPath : schemaPaths) {
+            InputStream resourceAsStream = getClass().getResourceAsStream(schemaPath);
+            schemas.add(resourceAsStream);
+        }
+
+        return schemas;
+    }
+
+    private SchemaContext parseSchemas(Collection<InputStream> schemas) throws IOException, YangSyntaxErrorException {
+        final YangParserImpl parser = new YangParserImpl();
+        Collection<ByteSource> sources = BuilderUtils.streamsToByteSources(schemas);
+        return parser.parseSources(sources);
+    }
+
+    private SchemaService createSchemaService() {
+        return new SchemaService() {
+
+            @Override
+            public void addModule(Module module) {
+            }
+
+            @Override
+            public void removeModule(Module module) {
+
+            }
+
+            @Override
+            public SchemaContext getSessionContext() {
+                return schemaContext;
+            }
+
+            @Override
+            public SchemaContext getGlobalContext() {
+                return schemaContext;
+            }
+
+            @Override
+            public ListenerRegistration<SchemaContextListener> registerSchemaContextListener(final SchemaContextListener listener) {
+                listener.onGlobalContextUpdated(getGlobalContext());
+                return new ListenerRegistration<SchemaContextListener>() {
+                    @Override
+                    public void close() {
+
+                    }
+
+                    @Override
+                    public SchemaContextListener getInstance() {
+                        return listener;
+                    }
+                };
+            }
+        };
+    }
+}
\ No newline at end of file
diff --git a/opendaylight/netconf/mdsal-netconf-connector/src/test/resources/messages/mapping/commit.xml b/opendaylight/netconf/mdsal-netconf-connector/src/test/resources/messages/mapping/commit.xml
new file mode 100644 (file)
index 0000000..615b4d7
--- /dev/null
@@ -0,0 +1,11 @@
+<!--
+  ~ Copyright (c) 2015 Cisco Systems, Inc. and others.  All rights reserved.
+  ~
+  ~ This program and the accompanying materials are made available under the
+  ~ terms of the Eclipse Public License v1.0 which accompanies this distribution,
+  ~ and is available at http://www.eclipse.org/legal/epl-v10.html
+  -->
+
+<rpc a="64" message-id="a" xmlns="urn:ietf:params:xml:ns:netconf:base:1.0">
+    <commit/>
+</rpc>
diff --git a/opendaylight/netconf/mdsal-netconf-connector/src/test/resources/messages/mapping/discardChanges.xml b/opendaylight/netconf/mdsal-netconf-connector/src/test/resources/messages/mapping/discardChanges.xml
new file mode 100644 (file)
index 0000000..d859091
--- /dev/null
@@ -0,0 +1,11 @@
+<!--
+  ~ Copyright (c) 2015 Cisco Systems, Inc. and others.  All rights reserved.
+  ~
+  ~ This program and the accompanying materials are made available under the
+  ~ terms of the Eclipse Public License v1.0 which accompanies this distribution,
+  ~ and is available at http://www.eclipse.org/legal/epl-v10.html
+  -->
+
+<rpc a="64" message-id="a" xmlns="urn:ietf:params:xml:ns:netconf:base:1.0">
+    <discard-changes/>
+</rpc>
diff --git a/opendaylight/netconf/mdsal-netconf-connector/src/test/resources/messages/mapping/editConfig_create.xml b/opendaylight/netconf/mdsal-netconf-connector/src/test/resources/messages/mapping/editConfig_create.xml
new file mode 100644 (file)
index 0000000..0f58c08
--- /dev/null
@@ -0,0 +1,27 @@
+<!--
+  ~ Copyright (c) 2015 Cisco Systems, Inc. and others.  All rights reserved.
+  ~
+  ~ This program and the accompanying materials are made available under the
+  ~ terms of the Eclipse Public License v1.0 which accompanies this distribution,
+  ~ and is available at http://www.eclipse.org/legal/epl-v10.html
+  -->
+
+<rpc message-id="a" a="64" xmlns="urn:ietf:params:xml:ns:netconf:base:1.0">
+    <edit-config>
+        <target>
+            <candidate/>
+        </target>
+        <test-option>
+            set
+        </test-option>
+        <default-operation>none</default-operation>
+        <config>
+            <mapping-nodes xmlns="urn:opendaylight:mdsal:mapping:test" xmlns:a="urn:ietf:params:xml:ns:netconf:base:1.0" a:operation="create">
+                <mapping-node>
+                    <id>node1-put</id>
+                    <content>put content</content>
+                </mapping-node>
+            </mapping-nodes>
+        </config>
+    </edit-config>
+</rpc>
\ No newline at end of file
diff --git a/opendaylight/netconf/mdsal-netconf-connector/src/test/resources/messages/mapping/editConfig_delete-root.xml b/opendaylight/netconf/mdsal-netconf-connector/src/test/resources/messages/mapping/editConfig_delete-root.xml
new file mode 100644 (file)
index 0000000..8f58768
--- /dev/null
@@ -0,0 +1,23 @@
+<!--
+  ~ Copyright (c) 2015 Cisco Systems, Inc. and others.  All rights reserved.
+  ~
+  ~ This program and the accompanying materials are made available under the
+  ~ terms of the Eclipse Public License v1.0 which accompanies this distribution,
+  ~ and is available at http://www.eclipse.org/legal/epl-v10.html
+  -->
+
+<rpc message-id="a" a="64" xmlns="urn:ietf:params:xml:ns:netconf:base:1.0">
+<edit-config>
+    <target>
+        <candidate/>
+    </target>
+    <test-option>
+        set
+    </test-option>
+    <default-operation>none</default-operation>
+    <config>
+        <mapping-nodes xmlns="urn:opendaylight:mdsal:mapping:test" xmlns:a="urn:ietf:params:xml:ns:netconf:base:1.0" a:operation="delete">
+        </mapping-nodes>
+    </config>
+</edit-config>
+</rpc>
\ No newline at end of file
diff --git a/opendaylight/netconf/mdsal-netconf-connector/src/test/resources/messages/mapping/editConfig_merge_missing_default-operation_1.xml b/opendaylight/netconf/mdsal-netconf-connector/src/test/resources/messages/mapping/editConfig_merge_missing_default-operation_1.xml
new file mode 100644 (file)
index 0000000..f4cbef5
--- /dev/null
@@ -0,0 +1,26 @@
+<!--
+  ~ Copyright (c) 2015 Cisco Systems, Inc. and others.  All rights reserved.
+  ~
+  ~ This program and the accompanying materials are made available under the
+  ~ terms of the Eclipse Public License v1.0 which accompanies this distribution,
+  ~ and is available at http://www.eclipse.org/legal/epl-v10.html
+  -->
+
+<rpc message-id="a" a="64" xmlns="urn:ietf:params:xml:ns:netconf:base:1.0">
+    <edit-config>
+        <target>
+            <candidate/>
+        </target>
+        <test-option>
+            set
+        </test-option>
+        <config>
+            <mapping-nodes xmlns="urn:opendaylight:mdsal:mapping:test">
+                <mapping-node>
+                    <id>node1-merge</id>
+                    <content>merged content</content>
+                </mapping-node>
+            </mapping-nodes>
+        </config>
+    </edit-config>
+</rpc>
\ No newline at end of file
diff --git a/opendaylight/netconf/mdsal-netconf-connector/src/test/resources/messages/mapping/editConfig_merge_missing_default-operation_2.xml b/opendaylight/netconf/mdsal-netconf-connector/src/test/resources/messages/mapping/editConfig_merge_missing_default-operation_2.xml
new file mode 100644 (file)
index 0000000..67561e6
--- /dev/null
@@ -0,0 +1,30 @@
+<!--
+  ~ Copyright (c) 2015 Cisco Systems, Inc. and others.  All rights reserved.
+  ~
+  ~ This program and the accompanying materials are made available under the
+  ~ terms of the Eclipse Public License v1.0 which accompanies this distribution,
+  ~ and is available at http://www.eclipse.org/legal/epl-v10.html
+  -->
+
+<rpc message-id="a" a="64" xmlns="urn:ietf:params:xml:ns:netconf:base:1.0">
+    <edit-config>
+        <target>
+            <candidate/>
+        </target>
+        <test-option>
+            set
+        </test-option>
+        <config>
+            <mapping-nodes xmlns="urn:opendaylight:mdsal:mapping:test">
+                <mapping-node>
+                    <id>node1-merge</id>
+                    <content>updated merged content</content>
+                </mapping-node>
+                <mapping-node>
+                    <id>node2-merge</id>
+                    <content>new node</content>
+                </mapping-node>
+            </mapping-nodes>
+        </config>
+    </edit-config>
+</rpc>
\ No newline at end of file
diff --git a/opendaylight/netconf/mdsal-netconf-connector/src/test/resources/messages/mapping/editConfig_merge_missing_default-operation_control.xml b/opendaylight/netconf/mdsal-netconf-connector/src/test/resources/messages/mapping/editConfig_merge_missing_default-operation_control.xml
new file mode 100644 (file)
index 0000000..44275aa
--- /dev/null
@@ -0,0 +1,22 @@
+<!--
+  ~ Copyright (c) 2015 Cisco Systems, Inc. and others.  All rights reserved.
+  ~
+  ~ This program and the accompanying materials are made available under the
+  ~ terms of the Eclipse Public License v1.0 which accompanies this distribution,
+  ~ and is available at http://www.eclipse.org/legal/epl-v10.html
+  -->
+
+<rpc-reply a="64" id="a" message-id="101" xmlns="urn:ietf:params:xml:ns:netconf:base:1.0" xmlnx="a:b:c:d">
+    <data xmlns="urn:ietf:params:xml:ns:netconf:base:1.0">
+        <mapping-nodes xmlns="urn:opendaylight:mdsal:mapping:test">
+            <mapping-node>
+                <id>node1-merge</id>
+                <content>updated merged content</content>
+            </mapping-node>
+            <mapping-node>
+                <id>node2-merge</id>
+                <content>new node</content>
+            </mapping-node>
+        </mapping-nodes>
+    </data>
+</rpc-reply>
\ No newline at end of file
diff --git a/opendaylight/netconf/mdsal-netconf-connector/src/test/resources/messages/mapping/editConfig_merge_multiple_1.xml b/opendaylight/netconf/mdsal-netconf-connector/src/test/resources/messages/mapping/editConfig_merge_multiple_1.xml
new file mode 100644 (file)
index 0000000..d117e9e
--- /dev/null
@@ -0,0 +1,31 @@
+<!--
+  ~ Copyright (c) 2015 Cisco Systems, Inc. and others.  All rights reserved.
+  ~
+  ~ This program and the accompanying materials are made available under the
+  ~ terms of the Eclipse Public License v1.0 which accompanies this distribution,
+  ~ and is available at http://www.eclipse.org/legal/epl-v10.html
+  -->
+
+<rpc message-id="a" a="64" xmlns="urn:ietf:params:xml:ns:netconf:base:1.0">
+    <edit-config>
+        <target>
+            <candidate/>
+        </target>
+        <test-option>
+            set
+        </test-option>
+        <default-operation>merge</default-operation>
+        <config>
+            <mapping-nodes xmlns="urn:opendaylight:mdsal:mapping:test">
+                <mapping-node>
+                    <id>node1-merge</id>
+                    <content>merged content node 1</content>
+                </mapping-node>
+                <mapping-node>
+                    <id>node2-merge</id>
+                    <content>merged content node 2</content>
+                </mapping-node>
+            </mapping-nodes>
+        </config>
+    </edit-config>
+</rpc>
\ No newline at end of file
diff --git a/opendaylight/netconf/mdsal-netconf-connector/src/test/resources/messages/mapping/editConfig_merge_multiple_2.xml b/opendaylight/netconf/mdsal-netconf-connector/src/test/resources/messages/mapping/editConfig_merge_multiple_2.xml
new file mode 100644 (file)
index 0000000..2cee889
--- /dev/null
@@ -0,0 +1,39 @@
+<!--
+  ~ Copyright (c) 2015 Cisco Systems, Inc. and others.  All rights reserved.
+  ~
+  ~ This program and the accompanying materials are made available under the
+  ~ terms of the Eclipse Public License v1.0 which accompanies this distribution,
+  ~ and is available at http://www.eclipse.org/legal/epl-v10.html
+  -->
+
+<rpc message-id="a" a="64" xmlns="urn:ietf:params:xml:ns:netconf:base:1.0">
+    <edit-config>
+        <target>
+            <candidate/>
+        </target>
+        <test-option>
+            set
+        </test-option>
+        <default-operation>merge</default-operation>
+        <config>
+            <mapping-nodes xmlns="urn:opendaylight:mdsal:mapping:test">
+                <mapping-node>
+                    <id>node1-merge</id>
+                    <content>overwritten old content node1</content>
+                </mapping-node>
+                <mapping-node>
+                    <id>node2-merge</id>
+                    <content>overwritten old content node2</content>
+                </mapping-node>
+                <mapping-node>
+                    <id>new-node4</id>
+                    <content>new node4 content</content>
+                </mapping-node>
+                <mapping-node xmlns:a="urn:ietf:params:xml:ns:netconf:base:1.0" a:operation="delete">
+                    <id>node3-merge</id>
+                    <content>merged content node 3</content>
+                </mapping-node>
+            </mapping-nodes>
+        </config>
+    </edit-config>
+</rpc>
\ No newline at end of file
diff --git a/opendaylight/netconf/mdsal-netconf-connector/src/test/resources/messages/mapping/editConfig_merge_multiple_3.xml b/opendaylight/netconf/mdsal-netconf-connector/src/test/resources/messages/mapping/editConfig_merge_multiple_3.xml
new file mode 100644 (file)
index 0000000..d6f18ab
--- /dev/null
@@ -0,0 +1,39 @@
+<!--
+  ~ Copyright (c) 2015 Cisco Systems, Inc. and others.  All rights reserved.
+  ~
+  ~ This program and the accompanying materials are made available under the
+  ~ terms of the Eclipse Public License v1.0 which accompanies this distribution,
+  ~ and is available at http://www.eclipse.org/legal/epl-v10.html
+  -->
+
+<rpc message-id="a" a="64" xmlns="urn:ietf:params:xml:ns:netconf:base:1.0">
+    <edit-config>
+        <target>
+            <candidate/>
+        </target>
+        <test-option>
+            set
+        </test-option>
+        <default-operation>merge</default-operation>
+        <config>
+            <mapping-nodes xmlns="urn:opendaylight:mdsal:mapping:test">
+                <mapping-node xmlns:a="urn:ietf:params:xml:ns:netconf:base:1.0" a:operation="delete">
+                    <id>node1-merge</id>
+                    <content>overwritten old content node1</content>
+                </mapping-node>
+                <mapping-node xmlns:a="urn:ietf:params:xml:ns:netconf:base:1.0" a:operation="delete">
+                    <id>node2-merge</id>
+                    <content>overwritten old content node2</content>
+                </mapping-node>
+                <mapping-node>
+                    <id>new-node5</id>
+                    <content>new node5 content</content>
+                </mapping-node>
+                <mapping-node>
+                    <id>new-node6</id>
+                    <content>new node6 content</content>
+                </mapping-node>
+            </mapping-nodes>
+        </config>
+    </edit-config>
+</rpc>
\ No newline at end of file
diff --git a/opendaylight/netconf/mdsal-netconf-connector/src/test/resources/messages/mapping/editConfig_merge_multiple_4_replace.xml b/opendaylight/netconf/mdsal-netconf-connector/src/test/resources/messages/mapping/editConfig_merge_multiple_4_replace.xml
new file mode 100644 (file)
index 0000000..55101fa
--- /dev/null
@@ -0,0 +1,27 @@
+<!--
+  ~ Copyright (c) 2015 Cisco Systems, Inc. and others.  All rights reserved.
+  ~
+  ~ This program and the accompanying materials are made available under the
+  ~ terms of the Eclipse Public License v1.0 which accompanies this distribution,
+  ~ and is available at http://www.eclipse.org/legal/epl-v10.html
+  -->
+
+<rpc message-id="a" a="64" xmlns="urn:ietf:params:xml:ns:netconf:base:1.0">
+    <edit-config>
+        <target>
+            <candidate/>
+        </target>
+        <test-option>
+            set
+        </test-option>
+        <default-operation>merge</default-operation>
+        <config>
+            <mapping-nodes xmlns="urn:opendaylight:mdsal:mapping:test" xmlns:a="urn:ietf:params:xml:ns:netconf:base:1.0" a:operation="replace">
+                <mapping-node>
+                    <id>new-node7</id>
+                    <content>new node content</content>
+                </mapping-node>
+            </mapping-nodes>
+        </config>
+    </edit-config>
+</rpc>
\ No newline at end of file
diff --git a/opendaylight/netconf/mdsal-netconf-connector/src/test/resources/messages/mapping/editConfig_merge_multiple_after_more_complex_merge.xml b/opendaylight/netconf/mdsal-netconf-connector/src/test/resources/messages/mapping/editConfig_merge_multiple_after_more_complex_merge.xml
new file mode 100644 (file)
index 0000000..496cfdb
--- /dev/null
@@ -0,0 +1,26 @@
+<!--
+  ~ Copyright (c) 2015 Cisco Systems, Inc. and others.  All rights reserved.
+  ~
+  ~ This program and the accompanying materials are made available under the
+  ~ terms of the Eclipse Public License v1.0 which accompanies this distribution,
+  ~ and is available at http://www.eclipse.org/legal/epl-v10.html
+  -->
+
+<rpc-reply a="64" id="a" message-id="101" xmlns="urn:ietf:params:xml:ns:netconf:base:1.0" xmlnx="a:b:c:d">
+<data xmlns="urn:ietf:params:xml:ns:netconf:base:1.0">
+    <mapping-nodes xmlns="urn:opendaylight:mdsal:mapping:test">
+        <mapping-node>
+            <id>node1-merge</id>
+            <content>overwritten old content node1</content>
+        </mapping-node>
+        <mapping-node>
+            <id>node2-merge</id>
+            <content>overwritten old content node2</content>
+        </mapping-node>
+        <mapping-node>
+            <id>new-node4</id>
+            <content>new node4 content</content>
+        </mapping-node>
+    </mapping-nodes>
+</data>
+</rpc-reply>
\ No newline at end of file
diff --git a/opendaylight/netconf/mdsal-netconf-connector/src/test/resources/messages/mapping/editConfig_merge_multiple_after_more_complex_merge_2.xml b/opendaylight/netconf/mdsal-netconf-connector/src/test/resources/messages/mapping/editConfig_merge_multiple_after_more_complex_merge_2.xml
new file mode 100644 (file)
index 0000000..f1142dc
--- /dev/null
@@ -0,0 +1,26 @@
+<!--
+  ~ Copyright (c) 2015 Cisco Systems, Inc. and others.  All rights reserved.
+  ~
+  ~ This program and the accompanying materials are made available under the
+  ~ terms of the Eclipse Public License v1.0 which accompanies this distribution,
+  ~ and is available at http://www.eclipse.org/legal/epl-v10.html
+  -->
+
+<rpc-reply a="64" id="a" message-id="101" xmlns="urn:ietf:params:xml:ns:netconf:base:1.0" xmlnx="a:b:c:d">
+<data xmlns="urn:ietf:params:xml:ns:netconf:base:1.0">
+    <mapping-nodes xmlns="urn:opendaylight:mdsal:mapping:test">
+        <mapping-node>
+            <id>new-node4</id>
+            <content>new node4 content</content>
+        </mapping-node>
+        <mapping-node>
+            <id>new-node5</id>
+            <content>new node5 content</content>
+        </mapping-node>
+        <mapping-node>
+            <id>new-node6</id>
+            <content>new node6 content</content>
+        </mapping-node>
+    </mapping-nodes>
+</data>
+</rpc-reply>
\ No newline at end of file
diff --git a/opendaylight/netconf/mdsal-netconf-connector/src/test/resources/messages/mapping/editConfig_merge_multiple_after_replace.xml b/opendaylight/netconf/mdsal-netconf-connector/src/test/resources/messages/mapping/editConfig_merge_multiple_after_replace.xml
new file mode 100644 (file)
index 0000000..fcece32
--- /dev/null
@@ -0,0 +1,18 @@
+<!--
+  ~ Copyright (c) 2015 Cisco Systems, Inc. and others.  All rights reserved.
+  ~
+  ~ This program and the accompanying materials are made available under the
+  ~ terms of the Eclipse Public License v1.0 which accompanies this distribution,
+  ~ and is available at http://www.eclipse.org/legal/epl-v10.html
+  -->
+
+<rpc-reply a="64" id="a" message-id="101" xmlns="urn:ietf:params:xml:ns:netconf:base:1.0" xmlnx="a:b:c:d">
+<data xmlns="urn:ietf:params:xml:ns:netconf:base:1.0">
+    <mapping-nodes xmlns="urn:opendaylight:mdsal:mapping:test">
+        <mapping-node>
+            <id>new-node7</id>
+            <content>new node content</content>
+        </mapping-node>
+    </mapping-nodes>
+</data>
+</rpc-reply>
\ No newline at end of file
diff --git a/opendaylight/netconf/mdsal-netconf-connector/src/test/resources/messages/mapping/editConfig_merge_multiple_control_1.xml b/opendaylight/netconf/mdsal-netconf-connector/src/test/resources/messages/mapping/editConfig_merge_multiple_control_1.xml
new file mode 100644 (file)
index 0000000..30b9d1a
--- /dev/null
@@ -0,0 +1,22 @@
+<!--
+  ~ Copyright (c) 2015 Cisco Systems, Inc. and others.  All rights reserved.
+  ~
+  ~ This program and the accompanying materials are made available under the
+  ~ terms of the Eclipse Public License v1.0 which accompanies this distribution,
+  ~ and is available at http://www.eclipse.org/legal/epl-v10.html
+  -->
+
+<rpc-reply a="64" id="a" message-id="101" xmlns="urn:ietf:params:xml:ns:netconf:base:1.0" xmlnx="a:b:c:d">
+    <data xmlns="urn:ietf:params:xml:ns:netconf:base:1.0">
+        <mapping-nodes xmlns="urn:opendaylight:mdsal:mapping:test">
+            <mapping-node>
+                <id>node1-merge</id>
+                <content>merged content node 1</content>
+            </mapping-node>
+            <mapping-node>
+                <id>node2-merge</id>
+                <content>merged content node 2</content>
+            </mapping-node>
+        </mapping-nodes>
+    </data>
+</rpc-reply>
\ No newline at end of file
diff --git a/opendaylight/netconf/mdsal-netconf-connector/src/test/resources/messages/mapping/editConfig_merge_multiple_control_2.xml b/opendaylight/netconf/mdsal-netconf-connector/src/test/resources/messages/mapping/editConfig_merge_multiple_control_2.xml
new file mode 100644 (file)
index 0000000..f455eb2
--- /dev/null
@@ -0,0 +1,26 @@
+<!--
+  ~ Copyright (c) 2015 Cisco Systems, Inc. and others.  All rights reserved.
+  ~
+  ~ This program and the accompanying materials are made available under the
+  ~ terms of the Eclipse Public License v1.0 which accompanies this distribution,
+  ~ and is available at http://www.eclipse.org/legal/epl-v10.html
+  -->
+
+<rpc-reply a="64" id="a" message-id="101" xmlns="urn:ietf:params:xml:ns:netconf:base:1.0" xmlnx="a:b:c:d">
+<data xmlns="urn:ietf:params:xml:ns:netconf:base:1.0">
+    <mapping-nodes xmlns="urn:opendaylight:mdsal:mapping:test">
+        <mapping-node>
+            <id>node1-merge</id>
+            <content>merged content node 1</content>
+        </mapping-node>
+        <mapping-node>
+            <id>node2-merge</id>
+            <content>merged content node 2</content>
+        </mapping-node>
+        <mapping-node>
+            <id>node3-merge</id>
+            <content>merged content node 3</content>
+        </mapping-node>
+    </mapping-nodes>
+</data>
+</rpc-reply>
\ No newline at end of file
diff --git a/opendaylight/netconf/mdsal-netconf-connector/src/test/resources/messages/mapping/editConfig_merge_n1.xml b/opendaylight/netconf/mdsal-netconf-connector/src/test/resources/messages/mapping/editConfig_merge_n1.xml
new file mode 100644 (file)
index 0000000..f007fba
--- /dev/null
@@ -0,0 +1,27 @@
+<!--
+  ~ Copyright (c) 2015 Cisco Systems, Inc. and others.  All rights reserved.
+  ~
+  ~ This program and the accompanying materials are made available under the
+  ~ terms of the Eclipse Public License v1.0 which accompanies this distribution,
+  ~ and is available at http://www.eclipse.org/legal/epl-v10.html
+  -->
+
+<rpc message-id="a" a="64" xmlns="urn:ietf:params:xml:ns:netconf:base:1.0">
+    <edit-config>
+        <target>
+            <candidate/>
+        </target>
+        <test-option>
+            set
+        </test-option>
+        <default-operation>merge</default-operation>
+        <config>
+            <mapping-nodes xmlns="urn:opendaylight:mdsal:mapping:test">
+                <mapping-node>
+                    <id>node1-put</id>
+                    <content>put content</content>
+                </mapping-node>
+            </mapping-nodes>
+        </config>
+    </edit-config>
+</rpc>
\ No newline at end of file
diff --git a/opendaylight/netconf/mdsal-netconf-connector/src/test/resources/messages/mapping/editConfig_merge_n1_control.xml b/opendaylight/netconf/mdsal-netconf-connector/src/test/resources/messages/mapping/editConfig_merge_n1_control.xml
new file mode 100644 (file)
index 0000000..5c68043
--- /dev/null
@@ -0,0 +1,18 @@
+<!--
+  ~ Copyright (c) 2015 Cisco Systems, Inc. and others.  All rights reserved.
+  ~
+  ~ This program and the accompanying materials are made available under the
+  ~ terms of the Eclipse Public License v1.0 which accompanies this distribution,
+  ~ and is available at http://www.eclipse.org/legal/epl-v10.html
+  -->
+
+<rpc-reply a="64" id="a" message-id="101" xmlns="urn:ietf:params:xml:ns:netconf:base:1.0" xmlnx="a:b:c:d">
+    <data xmlns="urn:ietf:params:xml:ns:netconf:base:1.0">
+        <mapping-nodes xmlns="urn:opendaylight:mdsal:mapping:test">
+            <mapping-node>
+                <id>node1-put</id>
+                <content>put content</content>
+            </mapping-node>
+        </mapping-nodes>
+    </data>
+</rpc-reply>
\ No newline at end of file
diff --git a/opendaylight/netconf/mdsal-netconf-connector/src/test/resources/messages/mapping/editConfig_merge_single_1.xml b/opendaylight/netconf/mdsal-netconf-connector/src/test/resources/messages/mapping/editConfig_merge_single_1.xml
new file mode 100644 (file)
index 0000000..9ea7de1
--- /dev/null
@@ -0,0 +1,27 @@
+<!--
+  ~ Copyright (c) 2015 Cisco Systems, Inc. and others.  All rights reserved.
+  ~
+  ~ This program and the accompanying materials are made available under the
+  ~ terms of the Eclipse Public License v1.0 which accompanies this distribution,
+  ~ and is available at http://www.eclipse.org/legal/epl-v10.html
+  -->
+
+<rpc message-id="a" a="64" xmlns="urn:ietf:params:xml:ns:netconf:base:1.0">
+    <edit-config>
+        <target>
+            <candidate/>
+        </target>
+        <test-option>
+            set
+        </test-option>
+        <default-operation>merge</default-operation>
+        <config>
+            <mapping-nodes xmlns="urn:opendaylight:mdsal:mapping:test">
+                <mapping-node>
+                    <id>node3-merge</id>
+                    <content>merged content node 3</content>
+                </mapping-node>
+            </mapping-nodes>
+        </config>
+    </edit-config>
+</rpc>
\ No newline at end of file
diff --git a/opendaylight/netconf/mdsal-netconf-connector/src/test/resources/messages/mapping/editConfig_remove-root.xml b/opendaylight/netconf/mdsal-netconf-connector/src/test/resources/messages/mapping/editConfig_remove-root.xml
new file mode 100644 (file)
index 0000000..296fd0c
--- /dev/null
@@ -0,0 +1,23 @@
+<!--
+  ~ Copyright (c) 2015 Cisco Systems, Inc. and others.  All rights reserved.
+  ~
+  ~ This program and the accompanying materials are made available under the
+  ~ terms of the Eclipse Public License v1.0 which accompanies this distribution,
+  ~ and is available at http://www.eclipse.org/legal/epl-v10.html
+  -->
+
+<rpc message-id="a" a="64" xmlns="urn:ietf:params:xml:ns:netconf:base:1.0">
+<edit-config>
+    <target>
+        <candidate/>
+    </target>
+    <test-option>
+        set
+    </test-option>
+    <default-operation>none</default-operation>
+    <config>
+        <mapping-nodes xmlns="urn:opendaylight:mdsal:mapping:test" xmlns:a="urn:ietf:params:xml:ns:netconf:base:1.0" a:operation="remove">
+        </mapping-nodes>
+    </config>
+</edit-config>
+</rpc>
\ No newline at end of file
diff --git a/opendaylight/netconf/mdsal-netconf-connector/src/test/resources/messages/mapping/editConfig_replace_default.xml b/opendaylight/netconf/mdsal-netconf-connector/src/test/resources/messages/mapping/editConfig_replace_default.xml
new file mode 100644 (file)
index 0000000..4120bd1
--- /dev/null
@@ -0,0 +1,31 @@
+<!--
+  ~ Copyright (c) 2015 Cisco Systems, Inc. and others.  All rights reserved.
+  ~
+  ~ This program and the accompanying materials are made available under the
+  ~ terms of the Eclipse Public License v1.0 which accompanies this distribution,
+  ~ and is available at http://www.eclipse.org/legal/epl-v10.html
+  -->
+
+<rpc message-id="a" a="64" xmlns="urn:ietf:params:xml:ns:netconf:base:1.0">
+    <edit-config>
+        <target>
+            <candidate/>
+        </target>
+        <test-option>
+            set
+        </test-option>
+        <default-operation>replace</default-operation>
+        <config>
+            <mapping-nodes xmlns="urn:opendaylight:mdsal:mapping:test">
+                <mapping-node>
+                    <id>new-node1</id>
+                    <content>replaced node 1 content</content>
+                </mapping-node>
+                <mapping-node>
+                    <id>new-node2</id>
+                    <content>replaced node 2 content</content>
+                </mapping-node>
+            </mapping-nodes>
+        </config>
+    </edit-config>
+</rpc>
\ No newline at end of file
diff --git a/opendaylight/netconf/mdsal-netconf-connector/src/test/resources/messages/mapping/editConfig_replace_default_control.xml b/opendaylight/netconf/mdsal-netconf-connector/src/test/resources/messages/mapping/editConfig_replace_default_control.xml
new file mode 100644 (file)
index 0000000..c917073
--- /dev/null
@@ -0,0 +1,22 @@
+<!--
+  ~ Copyright (c) 2015 Cisco Systems, Inc. and others.  All rights reserved.
+  ~
+  ~ This program and the accompanying materials are made available under the
+  ~ terms of the Eclipse Public License v1.0 which accompanies this distribution,
+  ~ and is available at http://www.eclipse.org/legal/epl-v10.html
+  -->
+
+<rpc-reply a="64" id="a" message-id="101" xmlns="urn:ietf:params:xml:ns:netconf:base:1.0" xmlnx="a:b:c:d">
+    <data xmlns="urn:ietf:params:xml:ns:netconf:base:1.0">
+        <mapping-nodes xmlns="urn:opendaylight:mdsal:mapping:test">
+            <mapping-node>
+                <id>new-node1</id>
+                <content>replaced node 1 content</content>
+            </mapping-node>
+            <mapping-node>
+                <id>new-node2</id>
+                <content>replaced node 2 content</content>
+            </mapping-node>
+        </mapping-nodes>
+    </data>
+</rpc-reply>
\ No newline at end of file
diff --git a/opendaylight/netconf/mdsal-netconf-connector/src/test/resources/messages/mapping/editConfig_running.xml b/opendaylight/netconf/mdsal-netconf-connector/src/test/resources/messages/mapping/editConfig_running.xml
new file mode 100644 (file)
index 0000000..3939eb5
--- /dev/null
@@ -0,0 +1,27 @@
+<!--
+  ~ Copyright (c) 2015 Cisco Systems, Inc. and others.  All rights reserved.
+  ~
+  ~ This program and the accompanying materials are made available under the
+  ~ terms of the Eclipse Public License v1.0 which accompanies this distribution,
+  ~ and is available at http://www.eclipse.org/legal/epl-v10.html
+  -->
+
+<rpc message-id="a" a="64" xmlns="urn:ietf:params:xml:ns:netconf:base:1.0">
+    <edit-config>
+        <target>
+            <running/>
+        </target>
+        <test-option>
+            set
+        </test-option>
+        <default-operation>merge</default-operation>
+        <config>
+            <mapping-nodes xmlns="urn:opendaylight:mdsal:mapping:test">
+                <mapping-node>
+                    <id>node1-put</id>
+                    <content>put content</content>
+                </mapping-node>
+            </mapping-nodes>
+        </config>
+    </edit-config>
+</rpc>
\ No newline at end of file
diff --git a/opendaylight/netconf/mdsal-netconf-connector/src/test/resources/messages/mapping/get.xml b/opendaylight/netconf/mdsal-netconf-connector/src/test/resources/messages/mapping/get.xml
new file mode 100644 (file)
index 0000000..efffd43
--- /dev/null
@@ -0,0 +1,11 @@
+<!--
+  ~ Copyright (c) 2015 Cisco Systems, Inc. and others.  All rights reserved.
+  ~
+  ~ This program and the accompanying materials are made available under the
+  ~ terms of the Eclipse Public License v1.0 which accompanies this distribution,
+  ~ and is available at http://www.eclipse.org/legal/epl-v10.html
+  -->
+
+<rpc message-id="a" a="64" xmlns="urn:ietf:params:xml:ns:netconf:base:1.0">
+    <get/>
+</rpc>
\ No newline at end of file
diff --git a/opendaylight/netconf/mdsal-netconf-connector/src/test/resources/messages/mapping/getConfig.xml b/opendaylight/netconf/mdsal-netconf-connector/src/test/resources/messages/mapping/getConfig.xml
new file mode 100644 (file)
index 0000000..351ea85
--- /dev/null
@@ -0,0 +1,15 @@
+<!--
+  ~ Copyright (c) 2015 Cisco Systems, Inc. and others.  All rights reserved.
+  ~
+  ~ This program and the accompanying materials are made available under the
+  ~ terms of the Eclipse Public License v1.0 which accompanies this distribution,
+  ~ and is available at http://www.eclipse.org/legal/epl-v10.html
+  -->
+
+<rpc id="a" a="64" xmlnx="a:b:c:d" xmlns="urn:ietf:params:xml:ns:netconf:base:1.0" message-id="101">
+    <get-config>
+        <source>
+            <running/>
+        </source>
+    </get-config>
+</rpc>
diff --git a/opendaylight/netconf/mdsal-netconf-connector/src/test/resources/messages/mapping/getConfig_candidate.xml b/opendaylight/netconf/mdsal-netconf-connector/src/test/resources/messages/mapping/getConfig_candidate.xml
new file mode 100644 (file)
index 0000000..72a95b9
--- /dev/null
@@ -0,0 +1,15 @@
+<!--
+  ~ Copyright (c) 2015 Cisco Systems, Inc. and others.  All rights reserved.
+  ~
+  ~ This program and the accompanying materials are made available under the
+  ~ terms of the Eclipse Public License v1.0 which accompanies this distribution,
+  ~ and is available at http://www.eclipse.org/legal/epl-v10.html
+  -->
+
+<rpc id="a" a="64" xmlnx="a:b:c:d" xmlns="urn:ietf:params:xml:ns:netconf:base:1.0" message-id="101">
+    <get-config>
+        <source>
+            <candidate/>
+        </source>
+    </get-config>
+</rpc>
diff --git a/opendaylight/netconf/mdsal-netconf-connector/src/test/resources/messages/mapping/lock.xml b/opendaylight/netconf/mdsal-netconf-connector/src/test/resources/messages/mapping/lock.xml
new file mode 100644 (file)
index 0000000..41a3459
--- /dev/null
@@ -0,0 +1,15 @@
+<!--
+  ~ Copyright (c) 2015 Cisco Systems, Inc. and others.  All rights reserved.
+  ~
+  ~ This program and the accompanying materials are made available under the
+  ~ terms of the Eclipse Public License v1.0 which accompanies this distribution,
+  ~ and is available at http://www.eclipse.org/legal/epl-v10.html
+  -->
+
+<rpc a="64" message-id="a" xmlns="urn:ietf:params:xml:ns:netconf:base:1.0">
+    <lock>
+        <target>
+            <running/>
+        </target>
+    </lock>
+</rpc>
diff --git a/opendaylight/netconf/mdsal-netconf-connector/src/test/resources/messages/mapping/lock_candidate.xml b/opendaylight/netconf/mdsal-netconf-connector/src/test/resources/messages/mapping/lock_candidate.xml
new file mode 100644 (file)
index 0000000..e48531b
--- /dev/null
@@ -0,0 +1,15 @@
+<!--
+  ~ Copyright (c) 2015 Cisco Systems, Inc. and others.  All rights reserved.
+  ~
+  ~ This program and the accompanying materials are made available under the
+  ~ terms of the Eclipse Public License v1.0 which accompanies this distribution,
+  ~ and is available at http://www.eclipse.org/legal/epl-v10.html
+  -->
+
+<rpc a="64" message-id="a" xmlns="urn:ietf:params:xml:ns:netconf:base:1.0">
+    <lock>
+        <target>
+            <candidate/>
+        </target>
+    </lock>
+</rpc>
diff --git a/opendaylight/netconf/mdsal-netconf-connector/src/test/resources/messages/mapping/rpc-reply_ok.xml b/opendaylight/netconf/mdsal-netconf-connector/src/test/resources/messages/mapping/rpc-reply_ok.xml
new file mode 100644 (file)
index 0000000..883df82
--- /dev/null
@@ -0,0 +1,10 @@
+<!--
+  ~ Copyright (c) 2015 Cisco Systems, Inc. and others.  All rights reserved.
+  ~
+  ~ This program and the accompanying materials are made available under the
+  ~ terms of the Eclipse Public License v1.0 which accompanies this distribution,
+  ~ and is available at http://www.eclipse.org/legal/epl-v10.html
+  -->
+<rpc-reply xmlns="urn:ietf:params:xml:ns:netconf:base:1.0" a="64" message-id="a">
+    <ok/>
+</rpc-reply>
\ No newline at end of file
diff --git a/opendaylight/netconf/mdsal-netconf-connector/src/test/resources/messages/mapping/unlock.xml b/opendaylight/netconf/mdsal-netconf-connector/src/test/resources/messages/mapping/unlock.xml
new file mode 100644 (file)
index 0000000..ddc299d
--- /dev/null
@@ -0,0 +1,15 @@
+<!--
+  ~ Copyright (c) 2015 Cisco Systems, Inc. and others.  All rights reserved.
+  ~
+  ~ This program and the accompanying materials are made available under the
+  ~ terms of the Eclipse Public License v1.0 which accompanies this distribution,
+  ~ and is available at http://www.eclipse.org/legal/epl-v10.html
+  -->
+
+<rpc a="64" message-id="a" xmlns="urn:ietf:params:xml:ns:netconf:base:1.0">
+    <unlock>
+        <target>
+            <running/>
+        </target>
+    </unlock>
+</rpc>
diff --git a/opendaylight/netconf/mdsal-netconf-connector/src/test/resources/messages/mapping/unlock_candidate.xml b/opendaylight/netconf/mdsal-netconf-connector/src/test/resources/messages/mapping/unlock_candidate.xml
new file mode 100644 (file)
index 0000000..84c336d
--- /dev/null
@@ -0,0 +1,15 @@
+<!--
+  ~ Copyright (c) 2015 Cisco Systems, Inc. and others.  All rights reserved.
+  ~
+  ~ This program and the accompanying materials are made available under the
+  ~ terms of the Eclipse Public License v1.0 which accompanies this distribution,
+  ~ and is available at http://www.eclipse.org/legal/epl-v10.html
+  -->
+
+<rpc a="64" message-id="a" xmlns="urn:ietf:params:xml:ns:netconf:base:1.0">
+    <unlock>
+        <target>
+            <candidate/>
+        </target>
+    </unlock>
+</rpc>
diff --git a/opendaylight/netconf/mdsal-netconf-connector/src/test/resources/yang/mdsal-netconf-mapping-test.yang b/opendaylight/netconf/mdsal-netconf-connector/src/test/resources/yang/mdsal-netconf-mapping-test.yang
new file mode 100644 (file)
index 0000000..69a82ee
--- /dev/null
@@ -0,0 +1,21 @@
+module config {
+    yang-version 1;
+    namespace "urn:opendaylight:mdsal:mapping:test";
+    prefix "map";
+
+    revision "2015-02-26";
+
+    container mapping-nodes {
+
+        list mapping-node{
+            key "id";
+            leaf id {
+                type string;
+            }
+
+            leaf content {
+                type string;
+            }
+        }
+    }
+}
\ No newline at end of file
index 6e4c16d20fca36bf83c89b77c872aa1fe873c08b..32fc49cefac0eb68f2c188dac021bad843ce1b4a 100644 (file)
       <artifactId>config-api</artifactId>
       <scope>test</scope>
     </dependency>
-    <dependency>
+      <dependency>
+          <groupId>org.opendaylight.yangtools</groupId>
+          <artifactId>object-cache-guava</artifactId>
+      </dependency>
+      <dependency>
+          <groupId>org.opendaylight.yangtools</groupId>
+          <artifactId>mockito-configuration</artifactId>
+      </dependency>
+
+      <dependency>
       <groupId>${project.groupId}</groupId>
       <artifactId>config-manager</artifactId>
       <scope>test</scope>