[ONOS-7054] Implement prototype of ISSU protocol

Change-Id: Id543c0de9c97b68f977c824cbc987b35d81beb2d
diff --git a/core/store/dist/src/main/java/org/onosproject/store/cluster/impl/DistributedLeadershipStore.java b/core/store/dist/src/main/java/org/onosproject/store/cluster/impl/DistributedLeadershipStore.java
index 78d2d4e..2bb1ea3 100644
--- a/core/store/dist/src/main/java/org/onosproject/store/cluster/impl/DistributedLeadershipStore.java
+++ b/core/store/dist/src/main/java/org/onosproject/store/cluster/impl/DistributedLeadershipStore.java
@@ -15,14 +15,12 @@
  */
 package org.onosproject.store.cluster.impl;
 
-import static org.onlab.util.Tools.groupedThreads;
-import static org.slf4j.LoggerFactory.getLogger;
-
 import java.util.Map;
 import java.util.Objects;
 import java.util.concurrent.ExecutorService;
 import java.util.concurrent.Executors;
 import java.util.function.Consumer;
+import java.util.stream.Collectors;
 
 import com.google.common.collect.Maps;
 import org.apache.felix.scr.annotations.Activate;
@@ -37,13 +35,21 @@
 import org.onosproject.cluster.LeadershipStore;
 import org.onosproject.cluster.LeadershipStoreDelegate;
 import org.onosproject.cluster.NodeId;
+import org.onosproject.core.Version;
+import org.onosproject.core.VersionService;
 import org.onosproject.event.Change;
 import org.onosproject.store.AbstractStore;
 import org.onosproject.store.service.DistributedPrimitive.Status;
+import org.onosproject.store.service.CoordinationService;
 import org.onosproject.store.service.LeaderElector;
-import org.onosproject.store.service.StorageService;
+import org.onosproject.upgrade.UpgradeEvent;
+import org.onosproject.upgrade.UpgradeEventListener;
+import org.onosproject.upgrade.UpgradeService;
 import org.slf4j.Logger;
 
+import static org.onlab.util.Tools.groupedThreads;
+import static org.slf4j.LoggerFactory.getLogger;
+
 /**
  * Implementation of {@code LeadershipStore} that makes use of a {@link LeaderElector}
  * primitive.
@@ -54,25 +60,41 @@
     extends AbstractStore<LeadershipEvent, LeadershipStoreDelegate>
     implements LeadershipStore {
 
+    private static final char VERSION_SEP = '|';
+
     private final Logger log = getLogger(getClass());
 
     @Reference(cardinality = ReferenceCardinality.MANDATORY_UNARY)
     protected ClusterService clusterService;
 
     @Reference(cardinality = ReferenceCardinality.MANDATORY_UNARY)
-    protected StorageService storageService;
+    protected CoordinationService storageService;
+
+    @Reference(cardinality = ReferenceCardinality.MANDATORY_UNARY)
+    protected VersionService versionService;
+
+    @Reference(cardinality = ReferenceCardinality.MANDATORY_UNARY)
+    protected UpgradeService upgradeService;
 
     private ExecutorService statusChangeHandler;
     private NodeId localNodeId;
     private LeaderElector leaderElector;
     private final Map<String, Leadership> localLeaderCache = Maps.newConcurrentMap();
+    private final UpgradeEventListener upgradeListener = new InternalUpgradeEventListener();
 
     private final Consumer<Change<Leadership>> leadershipChangeListener =
             change -> {
                 Leadership oldValue = change.oldValue();
                 Leadership newValue = change.newValue();
+
+                // If the topic is not relevant to this version, skip the event.
+                if (!isLocalTopic(newValue.topic())) {
+                    return;
+                }
+
                 boolean leaderChanged = !Objects.equals(oldValue.leader(), newValue.leader());
                 boolean candidatesChanged = !Objects.equals(oldValue.candidates(), newValue.candidates());
+
                 LeadershipEvent.Type eventType = null;
                 if (leaderChanged && candidatesChanged) {
                     eventType = LeadershipEvent.Type.LEADER_AND_CANDIDATES_CHANGED;
@@ -83,7 +105,10 @@
                 if (!leaderChanged && candidatesChanged) {
                     eventType = LeadershipEvent.Type.CANDIDATES_CHANGED;
                 }
-                notifyDelegate(new LeadershipEvent(eventType, change.newValue()));
+                notifyDelegate(new LeadershipEvent(eventType, new Leadership(
+                        parseTopic(change.newValue().topic()),
+                        change.newValue().leader(),
+                        change.newValue().candidates())));
                 // Update local cache of currently held leaderships
                 if (Objects.equals(newValue.leaderNodeId(), localNodeId)) {
                     localLeaderCache.put(newValue.topic(), newValue);
@@ -101,18 +126,27 @@
             // Service Restored
             localLeaderCache.forEach((topic, leadership) -> leaderElector.run(topic, localNodeId));
             leaderElector.getLeaderships().forEach((topic, leadership) ->
-                    notifyDelegate(new LeadershipEvent(LeadershipEvent.Type.SERVICE_RESTORED, leadership)));
+                    notifyDelegate(new LeadershipEvent(
+                            LeadershipEvent.Type.SERVICE_RESTORED,
+                            new Leadership(
+                                    parseTopic(leadership.topic()),
+                                    leadership.leader(),
+                                    leadership.candidates()))));
         } else if (status == Status.SUSPENDED) {
             // Service Suspended
             localLeaderCache.forEach((topic, leadership) ->
-                    notifyDelegate(new LeadershipEvent(LeadershipEvent.Type.SERVICE_DISRUPTED, leadership)));
+                    notifyDelegate(new LeadershipEvent(
+                            LeadershipEvent.Type.SERVICE_DISRUPTED,
+                            new Leadership(
+                                    parseTopic(leadership.topic()),
+                                    leadership.leader(),
+                                    leadership.candidates()))));
         } else {
             // Should be only inactive state
             return;
         }
     }
 
-
     @Activate
     public void activate() {
         statusChangeHandler = Executors.newSingleThreadExecutor(
@@ -124,6 +158,7 @@
                       .asLeaderElector();
         leaderElector.addChangeListener(leadershipChangeListener);
         leaderElector.addStatusChangeListener(clientStatusListener);
+        upgradeService.addListener(upgradeListener);
         log.info("Started");
     }
 
@@ -131,18 +166,20 @@
     public void deactivate() {
         leaderElector.removeChangeListener(leadershipChangeListener);
         leaderElector.removeStatusChangeListener(clientStatusListener);
+        upgradeService.removeListener(upgradeListener);
         statusChangeHandler.shutdown();
         log.info("Stopped");
     }
 
     @Override
     public Leadership addRegistration(String topic) {
-        return leaderElector.run(topic, localNodeId);
+        leaderElector.run(getLocalTopic(topic), localNodeId);
+        return getLeadership(topic);
     }
 
     @Override
     public void removeRegistration(String topic) {
-        leaderElector.withdraw(topic);
+        leaderElector.withdraw(getLocalTopic(topic));
     }
 
     @Override
@@ -152,21 +189,108 @@
 
     @Override
     public boolean moveLeadership(String topic, NodeId toNodeId) {
-        return leaderElector.anoint(topic, toNodeId);
+        return leaderElector.anoint(getTopicFor(topic, toNodeId), toNodeId);
     }
 
     @Override
     public boolean makeTopCandidate(String topic, NodeId nodeId) {
-        return leaderElector.promote(topic, nodeId);
+        return leaderElector.promote(getTopicFor(topic, nodeId), nodeId);
     }
 
     @Override
     public Leadership getLeadership(String topic) {
-        return leaderElector.getLeadership(topic);
+        Leadership leadership = leaderElector.getLeadership(getActiveTopic(topic));
+        return leadership != null ? new Leadership(
+                parseTopic(leadership.topic()),
+                leadership.leader(),
+                leadership.candidates()) : null;
     }
 
     @Override
     public Map<String, Leadership> getLeaderships() {
-        return leaderElector.getLeaderships();
+        Map<String, Leadership> leaderships = leaderElector.getLeaderships();
+        return leaderships.entrySet().stream()
+                .filter(e -> isActiveTopic(e.getKey()))
+                .collect(Collectors.toMap(e -> parseTopic(e.getKey()),
+                        e -> new Leadership(parseTopic(e.getKey()), e.getValue().leader(), e.getValue().candidates())));
+    }
+
+    /**
+     * Returns a leader elector topic namespaced with the local node's version.
+     *
+     * @param topic the base topic
+     * @return a topic string namespaced with the local node's version
+     */
+    private String getLocalTopic(String topic) {
+        return topic + VERSION_SEP + versionService.version();
+    }
+
+    /**
+     * Returns a leader elector topic namespaced with the current cluster version.
+     *
+     * @param topic the base topic
+     * @return a topic string namespaced with the current cluster version
+     */
+    private String getActiveTopic(String topic) {
+        return topic + VERSION_SEP + upgradeService.getVersion();
+    }
+
+    /**
+     * Returns whether the given topic is a topic for the local version.
+     *
+     * @param topic the topic to check
+     * @return whether the given topic is relevant to the local version
+     */
+    private boolean isLocalTopic(String topic) {
+        return topic.endsWith(versionService.version().toString());
+    }
+
+    /**
+     * Returns whether the given topic is a topic for the current cluster version.
+     *
+     * @param topic the topic to check
+     * @return whether the given topic is relevant to the current cluster version
+     */
+    private boolean isActiveTopic(String topic) {
+        return topic.endsWith(VERSION_SEP + upgradeService.getVersion().toString());
+    }
+
+    /**
+     * Parses a topic string, returning the base topic.
+     *
+     * @param topic the topic string to parse
+     * @return the base topic string
+     */
+    private String parseTopic(String topic) {
+        return topic.substring(0, topic.lastIndexOf(VERSION_SEP));
+    }
+
+    /**
+     * Returns the versioned topic for the given node.
+     *
+     * @param topic the topic for the given node
+     * @param nodeId the node for which to return the namespaced topic
+     * @return the versioned topic for the given node
+     */
+    private String getTopicFor(String topic, NodeId nodeId) {
+        Version nodeVersion = clusterService.getVersion(nodeId);
+        return nodeVersion != null ? topic + VERSION_SEP + nodeVersion : topic + VERSION_SEP + versionService.version();
+    }
+
+    /**
+     * Internal upgrade event listener.
+     */
+    private class InternalUpgradeEventListener implements UpgradeEventListener {
+        @Override
+        public void event(UpgradeEvent event) {
+            if (event.type() == UpgradeEvent.Type.UPGRADED || event.type() == UpgradeEvent.Type.ROLLED_BACK) {
+                // Iterate through all current leaderships for the new version and trigger events.
+                for (Leadership leadership : getLeaderships().values()) {
+                    notifyDelegate(new LeadershipEvent(
+                            LeadershipEvent.Type.LEADER_AND_CANDIDATES_CHANGED,
+                            leadership));
+                }
+            }
+        }
     }
 }
diff --git a/core/store/dist/src/main/java/org/onosproject/store/cluster/messaging/impl/AbstractClusterCommunicationManager.java b/core/store/dist/src/main/java/org/onosproject/store/cluster/messaging/impl/AbstractClusterCommunicationManager.java
new file mode 100644
index 0000000..aa96131
--- /dev/null
+++ b/core/store/dist/src/main/java/org/onosproject/store/cluster/messaging/impl/AbstractClusterCommunicationManager.java
@@ -0,0 +1,347 @@
+/*
+ * Copyright 2017-present Open Networking Foundation
+ *
+ * Licensed under the Apache License, Version 2.0 (the "License");
+ * you may not use this file except in compliance with the License.
+ * You may obtain a copy of the License at
+ *
+ *     http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+package org.onosproject.store.cluster.messaging.impl;
+
+import java.util.Set;
+import java.util.concurrent.CompletableFuture;
+import java.util.concurrent.Executor;
+import java.util.concurrent.ExecutorService;
+import java.util.function.BiConsumer;
+import java.util.function.BiFunction;
+import java.util.function.Consumer;
+import java.util.function.Function;
+import java.util.stream.Collectors;
+
+import com.google.common.base.Objects;
+import com.google.common.base.Throwables;
+import org.apache.felix.scr.annotations.Activate;
+import org.apache.felix.scr.annotations.Component;
+import org.apache.felix.scr.annotations.Deactivate;
+import org.apache.felix.scr.annotations.Reference;
+import org.apache.felix.scr.annotations.ReferenceCardinality;
+import org.onlab.util.Tools;
+import org.onosproject.cluster.ControllerNode;
+import org.onosproject.cluster.UnifiedClusterService;
+import org.onosproject.cluster.NodeId;
+import org.onosproject.store.cluster.messaging.ClusterCommunicationService;
+import org.onosproject.store.cluster.messaging.ClusterMessage;
+import org.onosproject.store.cluster.messaging.ClusterMessageHandler;
+import org.onosproject.store.cluster.messaging.Endpoint;
+import org.onosproject.store.cluster.messaging.MessageSubject;
+import org.onosproject.store.cluster.messaging.MessagingService;
+import org.onosproject.utils.MeteringAgent;
+import org.slf4j.Logger;
+import org.slf4j.LoggerFactory;
+
+import static com.google.common.base.Preconditions.checkArgument;
+import static com.google.common.base.Preconditions.checkNotNull;
+import static org.onosproject.security.AppGuard.checkPermission;
+import static org.onosproject.security.AppPermission.Type.CLUSTER_WRITE;
+
+@Component(componentAbstract = true)
+public abstract class AbstractClusterCommunicationManager
+        implements ClusterCommunicationService {
+
+    private final Logger log = LoggerFactory.getLogger(getClass());
+
+    private final MeteringAgent subjectMeteringAgent = new MeteringAgent(PRIMITIVE_NAME, SUBJECT_PREFIX, true);
+    private final MeteringAgent endpointMeteringAgent = new MeteringAgent(PRIMITIVE_NAME, ENDPOINT_PREFIX, true);
+
+    private static final String PRIMITIVE_NAME = "clusterCommunication";
+    private static final String SUBJECT_PREFIX = "subject";
+    private static final String ENDPOINT_PREFIX = "endpoint";
+
+    private static final String SERIALIZING = "serialization";
+    private static final String DESERIALIZING = "deserialization";
+    private static final String NODE_PREFIX = "node:";
+    private static final String ROUND_TRIP_SUFFIX = ".rtt";
+    private static final String ONE_WAY_SUFFIX = ".oneway";
+
+    @Reference(cardinality = ReferenceCardinality.MANDATORY_UNARY)
+    protected UnifiedClusterService clusterService;
+
+    @Reference(cardinality = ReferenceCardinality.MANDATORY_UNARY)
+    protected MessagingService messagingService;
+
+    private NodeId localNodeId;
+
+    /**
+     * Returns the type for the given message subject.
+     *
+     * @param subject the type for the given message subject
+     * @return the message subject
+     */
+    protected abstract String getType(MessageSubject subject);
+
+    @Activate
+    public void activate() {
+        localNodeId = clusterService.getLocalNode().id();
+        log.info("Started");
+    }
+
+    @Deactivate
+    public void deactivate() {
+        log.info("Stopped");
+    }
+
+    @Override
+    public <M> void broadcast(M message,
+                              MessageSubject subject,
+                              Function<M, byte[]> encoder) {
+        checkPermission(CLUSTER_WRITE);
+        multicast(message,
+                  subject,
+                  encoder,
+                  clusterService.getNodes()
+                      .stream()
+                      .filter(node -> !Objects.equal(node, clusterService.getLocalNode()))
+                      .map(ControllerNode::id)
+                      .collect(Collectors.toSet()));
+    }
+
+    @Override
+    public <M> void broadcastIncludeSelf(M message,
+                                         MessageSubject subject,
+                                         Function<M, byte[]> encoder) {
+        checkPermission(CLUSTER_WRITE);
+        multicast(message,
+                  subject,
+                  encoder,
+                  clusterService.getNodes()
+                      .stream()
+                      .map(ControllerNode::id)
+                      .collect(Collectors.toSet()));
+    }
+
+    @Override
+    public <M> CompletableFuture<Void> unicast(M message,
+                                               MessageSubject subject,
+                                               Function<M, byte[]> encoder,
+                                               NodeId toNodeId) {
+        checkPermission(CLUSTER_WRITE);
+        try {
+            byte[] payload = new ClusterMessage(
+                    localNodeId,
+                    subject,
+                    timeFunction(encoder, subjectMeteringAgent, SERIALIZING).apply(message)
+                    ).getBytes();
+            return doUnicast(subject, payload, toNodeId);
+        } catch (Exception e) {
+            return Tools.exceptionalFuture(e);
+        }
+    }
+
+    @Override
+    public <M> void multicast(M message,
+                              MessageSubject subject,
+                              Function<M, byte[]> encoder,
+                              Set<NodeId> nodes) {
+        checkPermission(CLUSTER_WRITE);
+        byte[] payload = new ClusterMessage(
+                localNodeId,
+                subject,
+                timeFunction(encoder, subjectMeteringAgent, SERIALIZING).apply(message))
+                .getBytes();
+        nodes.forEach(nodeId -> doUnicast(subject, payload, nodeId));
+    }
+
+    @Override
+    public <M, R> CompletableFuture<R> sendAndReceive(M message,
+                                                      MessageSubject subject,
+                                                      Function<M, byte[]> encoder,
+                                                      Function<byte[], R> decoder,
+                                                      NodeId toNodeId) {
+        checkPermission(CLUSTER_WRITE);
+        try {
+            ClusterMessage envelope = new ClusterMessage(
+                    clusterService.getLocalNode().id(),
+                    subject,
+                    timeFunction(encoder, subjectMeteringAgent, SERIALIZING).
+                            apply(message));
+            return sendAndReceive(subject, envelope.getBytes(), toNodeId).
+                    thenApply(bytes -> timeFunction(decoder, subjectMeteringAgent, DESERIALIZING).apply(bytes));
+        } catch (Exception e) {
+            return Tools.exceptionalFuture(e);
+        }
+    }
+
+    private CompletableFuture<Void> doUnicast(MessageSubject subject, byte[] payload, NodeId toNodeId) {
+        ControllerNode node = clusterService.getNode(toNodeId);
+        checkArgument(node != null, "Unknown nodeId: %s", toNodeId);
+        Endpoint nodeEp = new Endpoint(node.ip(), node.tcpPort());
+        MeteringAgent.Context context = subjectMeteringAgent.startTimer(subject.toString() + ONE_WAY_SUFFIX);
+        return messagingService.sendAsync(nodeEp, getType(subject), payload).whenComplete((r, e) -> context.stop(e));
+    }
+
+    private CompletableFuture<byte[]> sendAndReceive(MessageSubject subject, byte[] payload, NodeId toNodeId) {
+        ControllerNode node = clusterService.getNode(toNodeId);
+        checkArgument(node != null, "Unknown nodeId: %s", toNodeId);
+        Endpoint nodeEp = new Endpoint(node.ip(), node.tcpPort());
+        MeteringAgent.Context epContext = endpointMeteringAgent.
+                startTimer(NODE_PREFIX + toNodeId.toString() + ROUND_TRIP_SUFFIX);
+        MeteringAgent.Context subjectContext = subjectMeteringAgent.
+                startTimer(subject.toString() + ROUND_TRIP_SUFFIX);
+        return messagingService.sendAndReceive(nodeEp, getType(subject), payload).
+                whenComplete((bytes, throwable) -> {
+                    subjectContext.stop(throwable);
+                    epContext.stop(throwable);
+                });
+    }
+
+    @Override
+    public void addSubscriber(MessageSubject subject,
+                              ClusterMessageHandler subscriber,
+                              ExecutorService executor) {
+        checkPermission(CLUSTER_WRITE);
+        messagingService.registerHandler(getType(subject),
+                new InternalClusterMessageHandler(subscriber),
+                executor);
+    }
+
+    @Override
+    public void removeSubscriber(MessageSubject subject) {
+        checkPermission(CLUSTER_WRITE);
+        messagingService.unregisterHandler(getType(subject));
+    }
+
+    @Override
+    public <M, R> void addSubscriber(MessageSubject subject,
+            Function<byte[], M> decoder,
+            Function<M, R> handler,
+            Function<R, byte[]> encoder,
+            Executor executor) {
+        checkPermission(CLUSTER_WRITE);
+        messagingService.registerHandler(getType(subject),
+                new InternalMessageResponder<M, R>(decoder, encoder, m -> {
+                    CompletableFuture<R> responseFuture = new CompletableFuture<>();
+                    executor.execute(() -> {
+                        try {
+                            responseFuture.complete(handler.apply(m));
+                        } catch (Exception e) {
+                            responseFuture.completeExceptionally(e);
+                        }
+                    });
+                    return responseFuture;
+                }));
+    }
+
+    @Override
+    public <M, R> void addSubscriber(MessageSubject subject,
+            Function<byte[], M> decoder,
+            Function<M, CompletableFuture<R>> handler,
+            Function<R, byte[]> encoder) {
+        checkPermission(CLUSTER_WRITE);
+        messagingService.registerHandler(getType(subject),
+                new InternalMessageResponder<>(decoder, encoder, handler));
+    }
+
+    @Override
+    public <M> void addSubscriber(MessageSubject subject,
+            Function<byte[], M> decoder,
+            Consumer<M> handler,
+            Executor executor) {
+        checkPermission(CLUSTER_WRITE);
+        messagingService.registerHandler(getType(subject),
+                new InternalMessageConsumer<>(decoder, handler),
+                executor);
+    }
+
+    /**
+     * Performs the timed function, returning the value it would while timing the operation.
+     *
+     * @param timedFunction the function to be timed
+     * @param meter the metering agent to be used to time the function
+     * @param opName the opname to be used when starting the meter
+     * @param <A> The param type of the function
+     * @param <B> The return type of the function
+     * @return the value returned by the timed function
+     */
+    private <A, B> Function<A, B> timeFunction(Function<A, B> timedFunction,
+                                               MeteringAgent meter, String opName) {
+        checkNotNull(timedFunction);
+        checkNotNull(meter);
+        checkNotNull(opName);
+        return new Function<A, B>() {
+            @Override
+            public B apply(A a) {
+                final MeteringAgent.Context context = meter.startTimer(opName);
+                B result = null;
+                try {
+                    result = timedFunction.apply(a);
+                    context.stop(null);
+                    return result;
+                } catch (Exception e) {
+                    context.stop(e);
+                    Throwables.propagate(e);
+                    return null;
+                }
+            }
+        };
+    }
+
+
+    private class InternalClusterMessageHandler implements BiFunction<Endpoint, byte[], byte[]> {
+        private ClusterMessageHandler handler;
+
+        public InternalClusterMessageHandler(ClusterMessageHandler handler) {
+            this.handler = handler;
+        }
+
+        @Override
+        public byte[] apply(Endpoint sender, byte[] bytes) {
+            ClusterMessage message = ClusterMessage.fromBytes(bytes);
+            handler.handle(message);
+            return message.response();
+        }
+    }
+
+    private class InternalMessageResponder<M, R> implements BiFunction<Endpoint, byte[], CompletableFuture<byte[]>> {
+        private final Function<byte[], M> decoder;
+        private final Function<R, byte[]> encoder;
+        private final Function<M, CompletableFuture<R>> handler;
+
+        public InternalMessageResponder(Function<byte[], M> decoder,
+                                        Function<R, byte[]> encoder,
+                                        Function<M, CompletableFuture<R>> handler) {
+            this.decoder = decoder;
+            this.encoder = encoder;
+            this.handler = handler;
+        }
+
+        @Override
+        public CompletableFuture<byte[]> apply(Endpoint sender, byte[] bytes) {
+            return handler.apply(timeFunction(decoder, subjectMeteringAgent, DESERIALIZING).
+                    apply(ClusterMessage.fromBytes(bytes).payload())).
+                    thenApply(m -> timeFunction(encoder, subjectMeteringAgent, SERIALIZING).apply(m));
+        }
+    }
+
+    private class InternalMessageConsumer<M> implements BiConsumer<Endpoint, byte[]> {
+        private final Function<byte[], M> decoder;
+        private final Consumer<M> consumer;
+
+        public InternalMessageConsumer(Function<byte[], M> decoder, Consumer<M> consumer) {
+            this.decoder = decoder;
+            this.consumer = consumer;
+        }
+
+        @Override
+        public void accept(Endpoint sender, byte[] bytes) {
+            consumer.accept(timeFunction(decoder, subjectMeteringAgent, DESERIALIZING).
+                    apply(ClusterMessage.fromBytes(bytes).payload()));
+        }
+    }
+}
diff --git a/core/store/dist/src/main/java/org/onosproject/store/cluster/messaging/impl/ClusterCommunicationManager.java b/core/store/dist/src/main/java/org/onosproject/store/cluster/messaging/impl/ClusterCommunicationManager.java
index 1bdac37..4602702 100644
--- a/core/store/dist/src/main/java/org/onosproject/store/cluster/messaging/impl/ClusterCommunicationManager.java
+++ b/core/store/dist/src/main/java/org/onosproject/store/cluster/messaging/impl/ClusterCommunicationManager.java
@@ -15,328 +15,24 @@
  */
 package org.onosproject.store.cluster.messaging.impl;
 
-import com.google.common.base.Throwables;
-import org.apache.felix.scr.annotations.Activate;
 import org.apache.felix.scr.annotations.Component;
-import org.apache.felix.scr.annotations.Deactivate;
 import org.apache.felix.scr.annotations.Reference;
 import org.apache.felix.scr.annotations.ReferenceCardinality;
 import org.apache.felix.scr.annotations.Service;
-import org.onlab.util.Tools;
-import org.onosproject.cluster.ClusterService;
-import org.onosproject.cluster.ControllerNode;
-import org.onosproject.cluster.NodeId;
-import org.onosproject.store.cluster.messaging.ClusterCommunicationService;
-import org.onosproject.store.cluster.messaging.ClusterMessage;
-import org.onosproject.store.cluster.messaging.ClusterMessageHandler;
-import org.onosproject.store.cluster.messaging.Endpoint;
+import org.onosproject.core.VersionService;
 import org.onosproject.store.cluster.messaging.MessageSubject;
-import org.onosproject.store.cluster.messaging.MessagingService;
-import org.onosproject.utils.MeteringAgent;
-import org.slf4j.Logger;
-import org.slf4j.LoggerFactory;
-
-import com.google.common.base.Objects;
-
-import java.util.Set;
-import java.util.concurrent.CompletableFuture;
-import java.util.concurrent.Executor;
-import java.util.concurrent.ExecutorService;
-import java.util.function.BiConsumer;
-import java.util.function.BiFunction;
-import java.util.function.Consumer;
-import java.util.function.Function;
-import java.util.stream.Collectors;
-
-import static com.google.common.base.Preconditions.checkArgument;
-import static com.google.common.base.Preconditions.checkNotNull;
-import static org.onosproject.security.AppGuard.checkPermission;
-import static org.onosproject.security.AppPermission.Type.CLUSTER_WRITE;
 
 @Component(immediate = true)
 @Service
-public class ClusterCommunicationManager
-        implements ClusterCommunicationService {
+public class ClusterCommunicationManager extends AbstractClusterCommunicationManager {
 
-    private final Logger log = LoggerFactory.getLogger(getClass());
-
-    private final MeteringAgent subjectMeteringAgent = new MeteringAgent(PRIMITIVE_NAME, SUBJECT_PREFIX, true);
-    private final MeteringAgent endpointMeteringAgent = new MeteringAgent(PRIMITIVE_NAME, ENDPOINT_PREFIX, true);
-
-    private static final String PRIMITIVE_NAME = "clusterCommunication";
-    private static final String SUBJECT_PREFIX = "subject";
-    private static final String ENDPOINT_PREFIX = "endpoint";
-
-    private static final String SERIALIZING = "serialization";
-    private static final String DESERIALIZING = "deserialization";
-    private static final String NODE_PREFIX = "node:";
-    private static final String ROUND_TRIP_SUFFIX = ".rtt";
-    private static final String ONE_WAY_SUFFIX = ".oneway";
+    private static final char VERSION_SEP = '-';
 
     @Reference(cardinality = ReferenceCardinality.MANDATORY_UNARY)
-    private ClusterService clusterService;
-
-    @Reference(cardinality = ReferenceCardinality.MANDATORY_UNARY)
-    protected MessagingService messagingService;
-
-    private NodeId localNodeId;
-
-    @Activate
-    public void activate() {
-        localNodeId = clusterService.getLocalNode().id();
-        log.info("Started");
-    }
-
-    @Deactivate
-    public void deactivate() {
-        log.info("Stopped");
-    }
+    private VersionService versionService;
 
     @Override
-    public <M> void broadcast(M message,
-                              MessageSubject subject,
-                              Function<M, byte[]> encoder) {
-        checkPermission(CLUSTER_WRITE);
-        multicast(message,
-                  subject,
-                  encoder,
-                  clusterService.getNodes()
-                      .stream()
-                      .filter(node -> !Objects.equal(node, clusterService.getLocalNode()))
-                      .map(ControllerNode::id)
-                      .collect(Collectors.toSet()));
-    }
-
-    @Override
-    public <M> void broadcastIncludeSelf(M message,
-                                         MessageSubject subject,
-                                         Function<M, byte[]> encoder) {
-        checkPermission(CLUSTER_WRITE);
-        multicast(message,
-                  subject,
-                  encoder,
-                  clusterService.getNodes()
-                      .stream()
-                      .map(ControllerNode::id)
-                      .collect(Collectors.toSet()));
-    }
-
-    @Override
-    public <M> CompletableFuture<Void> unicast(M message,
-                                               MessageSubject subject,
-                                               Function<M, byte[]> encoder,
-                                               NodeId toNodeId) {
-        checkPermission(CLUSTER_WRITE);
-        try {
-            byte[] payload = new ClusterMessage(
-                    localNodeId,
-                    subject,
-                    timeFunction(encoder, subjectMeteringAgent, SERIALIZING).apply(message)
-                    ).getBytes();
-            return doUnicast(subject, payload, toNodeId);
-        } catch (Exception e) {
-            return Tools.exceptionalFuture(e);
-        }
-    }
-
-    @Override
-    public <M> void multicast(M message,
-                              MessageSubject subject,
-                              Function<M, byte[]> encoder,
-                              Set<NodeId> nodes) {
-        checkPermission(CLUSTER_WRITE);
-        byte[] payload = new ClusterMessage(
-                localNodeId,
-                subject,
-                timeFunction(encoder, subjectMeteringAgent, SERIALIZING).apply(message))
-                .getBytes();
-        nodes.forEach(nodeId -> doUnicast(subject, payload, nodeId));
-    }
-
-    @Override
-    public <M, R> CompletableFuture<R> sendAndReceive(M message,
-                                                      MessageSubject subject,
-                                                      Function<M, byte[]> encoder,
-                                                      Function<byte[], R> decoder,
-                                                      NodeId toNodeId) {
-        checkPermission(CLUSTER_WRITE);
-        try {
-            ClusterMessage envelope = new ClusterMessage(
-                    clusterService.getLocalNode().id(),
-                    subject,
-                    timeFunction(encoder, subjectMeteringAgent, SERIALIZING).
-                            apply(message));
-            return sendAndReceive(subject, envelope.getBytes(), toNodeId).
-                    thenApply(bytes -> timeFunction(decoder, subjectMeteringAgent, DESERIALIZING).apply(bytes));
-        } catch (Exception e) {
-            return Tools.exceptionalFuture(e);
-        }
-    }
-
-    private CompletableFuture<Void> doUnicast(MessageSubject subject, byte[] payload, NodeId toNodeId) {
-        ControllerNode node = clusterService.getNode(toNodeId);
-        checkArgument(node != null, "Unknown nodeId: %s", toNodeId);
-        Endpoint nodeEp = new Endpoint(node.ip(), node.tcpPort());
-        MeteringAgent.Context context = subjectMeteringAgent.startTimer(subject.toString() + ONE_WAY_SUFFIX);
-        return messagingService.sendAsync(nodeEp, subject.value(), payload).whenComplete((r, e) -> context.stop(e));
-    }
-
-    private CompletableFuture<byte[]> sendAndReceive(MessageSubject subject, byte[] payload, NodeId toNodeId) {
-        ControllerNode node = clusterService.getNode(toNodeId);
-        checkArgument(node != null, "Unknown nodeId: %s", toNodeId);
-        Endpoint nodeEp = new Endpoint(node.ip(), node.tcpPort());
-        MeteringAgent.Context epContext = endpointMeteringAgent.
-                startTimer(NODE_PREFIX + toNodeId.toString() + ROUND_TRIP_SUFFIX);
-        MeteringAgent.Context subjectContext = subjectMeteringAgent.
-                startTimer(subject.toString() + ROUND_TRIP_SUFFIX);
-        return messagingService.sendAndReceive(nodeEp, subject.value(), payload).
-                whenComplete((bytes, throwable) -> {
-                    subjectContext.stop(throwable);
-                    epContext.stop(throwable);
-                });
-    }
-
-    @Override
-    public void addSubscriber(MessageSubject subject,
-                              ClusterMessageHandler subscriber,
-                              ExecutorService executor) {
-        checkPermission(CLUSTER_WRITE);
-        messagingService.registerHandler(subject.value(),
-                new InternalClusterMessageHandler(subscriber),
-                executor);
-    }
-
-    @Override
-    public void removeSubscriber(MessageSubject subject) {
-        checkPermission(CLUSTER_WRITE);
-        messagingService.unregisterHandler(subject.value());
-    }
-
-    @Override
-    public <M, R> void addSubscriber(MessageSubject subject,
-            Function<byte[], M> decoder,
-            Function<M, R> handler,
-            Function<R, byte[]> encoder,
-            Executor executor) {
-        checkPermission(CLUSTER_WRITE);
-        messagingService.registerHandler(subject.value(),
-                new InternalMessageResponder<M, R>(decoder, encoder, m -> {
-                    CompletableFuture<R> responseFuture = new CompletableFuture<>();
-                    executor.execute(() -> {
-                        try {
-                            responseFuture.complete(handler.apply(m));
-                        } catch (Exception e) {
-                            responseFuture.completeExceptionally(e);
-                        }
-                    });
-                    return responseFuture;
-                }));
-    }
-
-    @Override
-    public <M, R> void addSubscriber(MessageSubject subject,
-            Function<byte[], M> decoder,
-            Function<M, CompletableFuture<R>> handler,
-            Function<R, byte[]> encoder) {
-        checkPermission(CLUSTER_WRITE);
-        messagingService.registerHandler(subject.value(),
-                new InternalMessageResponder<>(decoder, encoder, handler));
-    }
-
-    @Override
-    public <M> void addSubscriber(MessageSubject subject,
-            Function<byte[], M> decoder,
-            Consumer<M> handler,
-            Executor executor) {
-        checkPermission(CLUSTER_WRITE);
-        messagingService.registerHandler(subject.value(),
-                new InternalMessageConsumer<>(decoder, handler),
-                executor);
-    }
-
-    /**
-     * Performs the timed function, returning the value it would while timing the operation.
-     *
-     * @param timedFunction the function to be timed
-     * @param meter the metering agent to be used to time the function
-     * @param opName the opname to be used when starting the meter
-     * @param <A> The param type of the function
-     * @param <B> The return type of the function
-     * @return the value returned by the timed function
-     */
-    private <A, B> Function<A, B> timeFunction(Function<A, B> timedFunction,
-                                               MeteringAgent meter, String opName) {
-        checkNotNull(timedFunction);
-        checkNotNull(meter);
-        checkNotNull(opName);
-        return new Function<A, B>() {
-            @Override
-            public B apply(A a) {
-                final MeteringAgent.Context context = meter.startTimer(opName);
-                B result = null;
-                try {
-                    result = timedFunction.apply(a);
-                    context.stop(null);
-                    return result;
-                } catch (Exception e) {
-                    context.stop(e);
-                    Throwables.propagate(e);
-                    return null;
-                }
-            }
-        };
-    }
-
-
-    private class InternalClusterMessageHandler implements BiFunction<Endpoint, byte[], byte[]> {
-        private ClusterMessageHandler handler;
-
-        public InternalClusterMessageHandler(ClusterMessageHandler handler) {
-            this.handler = handler;
-        }
-
-        @Override
-        public byte[] apply(Endpoint sender, byte[] bytes) {
-            ClusterMessage message = ClusterMessage.fromBytes(bytes);
-            handler.handle(message);
-            return message.response();
-        }
-    }
-
-    private class InternalMessageResponder<M, R> implements BiFunction<Endpoint, byte[], CompletableFuture<byte[]>> {
-        private final Function<byte[], M> decoder;
-        private final Function<R, byte[]> encoder;
-        private final Function<M, CompletableFuture<R>> handler;
-
-        public InternalMessageResponder(Function<byte[], M> decoder,
-                                        Function<R, byte[]> encoder,
-                                        Function<M, CompletableFuture<R>> handler) {
-            this.decoder = decoder;
-            this.encoder = encoder;
-            this.handler = handler;
-        }
-
-        @Override
-        public CompletableFuture<byte[]> apply(Endpoint sender, byte[] bytes) {
-            return handler.apply(timeFunction(decoder, subjectMeteringAgent, DESERIALIZING).
-                    apply(ClusterMessage.fromBytes(bytes).payload())).
-                    thenApply(m -> timeFunction(encoder, subjectMeteringAgent, SERIALIZING).apply(m));
-        }
-    }
-
-    private class InternalMessageConsumer<M> implements BiConsumer<Endpoint, byte[]> {
-        private final Function<byte[], M> decoder;
-        private final Consumer<M> consumer;
-
-        public InternalMessageConsumer(Function<byte[], M> decoder, Consumer<M> consumer) {
-            this.decoder = decoder;
-            this.consumer = consumer;
-        }
-
-        @Override
-        public void accept(Endpoint sender, byte[] bytes) {
-            consumer.accept(timeFunction(decoder, subjectMeteringAgent, DESERIALIZING).
-                    apply(ClusterMessage.fromBytes(bytes).payload()));
-        }
+    protected String getType(MessageSubject subject) {
+        return subject.value() + VERSION_SEP + versionService.version().toString();
     }
 }
diff --git a/core/store/dist/src/main/java/org/onosproject/store/cluster/messaging/impl/UnifiedClusterCommunicationManager.java b/core/store/dist/src/main/java/org/onosproject/store/cluster/messaging/impl/UnifiedClusterCommunicationManager.java
new file mode 100644
index 0000000..45c84af
--- /dev/null
+++ b/core/store/dist/src/main/java/org/onosproject/store/cluster/messaging/impl/UnifiedClusterCommunicationManager.java
@@ -0,0 +1,32 @@
+/*
+ * Copyright 2017-present Open Networking Foundation
+ *
+ * Licensed under the Apache License, Version 2.0 (the "License");
+ * you may not use this file except in compliance with the License.
+ * You may obtain a copy of the License at
+ *
+ *     http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+package org.onosproject.store.cluster.messaging.impl;
+
+import org.apache.felix.scr.annotations.Component;
+import org.apache.felix.scr.annotations.Service;
+import org.onosproject.store.cluster.messaging.UnifiedClusterCommunicationService;
+import org.onosproject.store.cluster.messaging.MessageSubject;
+
+@Component(immediate = true)
+@Service
+public class UnifiedClusterCommunicationManager
+        extends AbstractClusterCommunicationManager
+        implements UnifiedClusterCommunicationService {
+    @Override
+    protected String getType(MessageSubject subject) {
+        return subject.value();
+    }
+}