[ONOS-5729] VirtualFlowRuleStore implementation

Changes
1. VirtualStore and AbstractVirtualStore are defined
2. SimpleFlowRuleStore is added
3. FlowRuleStoreDeligate is implemented

Change-Id: Ic1f3d25f8a52d8c9bb6bc4de67abc0d642f47c9c
diff --git a/incubator/api/src/main/java/org/onosproject/incubator/net/virtual/VirtualNetworkFlowRuleStore.java b/incubator/api/src/main/java/org/onosproject/incubator/net/virtual/VirtualNetworkFlowRuleStore.java
index faca465..0567083 100644
--- a/incubator/api/src/main/java/org/onosproject/incubator/net/virtual/VirtualNetworkFlowRuleStore.java
+++ b/incubator/api/src/main/java/org/onosproject/incubator/net/virtual/VirtualNetworkFlowRuleStore.java
@@ -24,12 +24,11 @@
 import org.onosproject.net.flow.FlowRuleEvent;
 import org.onosproject.net.flow.FlowRuleStoreDelegate;
 import org.onosproject.net.flow.TableStatisticsEntry;
-import org.onosproject.store.Store;
 
 import java.util.List;
 
 public interface VirtualNetworkFlowRuleStore
-        extends Store<FlowRuleBatchEvent, FlowRuleStoreDelegate> {
+        extends VirtualStore<FlowRuleBatchEvent, FlowRuleStoreDelegate> {
     /**
      * Returns the number of flow rule in the store.
      *
diff --git a/incubator/api/src/main/java/org/onosproject/incubator/net/virtual/VirtualStore.java b/incubator/api/src/main/java/org/onosproject/incubator/net/virtual/VirtualStore.java
new file mode 100644
index 0000000..95249f4
--- /dev/null
+++ b/incubator/api/src/main/java/org/onosproject/incubator/net/virtual/VirtualStore.java
@@ -0,0 +1,54 @@
+/*
+ * Copyright 2016-present Open Networking Laboratory
+ *
+ * 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.incubator.net.virtual;
+
+import org.onosproject.event.Event;
+import org.onosproject.store.StoreDelegate;
+
+/**
+ * Abstraction of a entity capable of storing and/or distributing information
+ * for virtual network across a cluster.
+ */
+public interface VirtualStore<E extends Event, D extends StoreDelegate<E>> {
+    /**
+     * Sets the delegate on the store.
+     *
+     * @param networkId a virtual network identifier
+     * @param delegate new store delegate
+     * @throws java.lang.IllegalStateException if a delegate is already
+     *                                         currently set on the store and is a different one that
+     */
+    void setDelegate(NetworkId networkId, D delegate);
+
+    /**
+     * Withdraws the delegate from the store.
+     *
+     * @param networkId a virtual network identifier
+     * @param delegate store delegate to withdraw
+     * @throws java.lang.IllegalArgumentException if the delegate is not
+     *                                            currently set on the store
+     */
+    void unsetDelegate(NetworkId networkId, D delegate);
+
+    /**
+     * Indicates whether the store has a delegate.
+     *
+     * @param networkId a virtual network identifier
+     * @return true if delegate is set
+     */
+    boolean hasDelegate(NetworkId networkId);
+}
diff --git a/incubator/net/src/main/java/org/onosproject/incubator/net/virtual/impl/VirtualNetworkFlowRuleManager.java b/incubator/net/src/main/java/org/onosproject/incubator/net/virtual/impl/VirtualNetworkFlowRuleManager.java
index 0ac256e..31c1415 100644
--- a/incubator/net/src/main/java/org/onosproject/incubator/net/virtual/impl/VirtualNetworkFlowRuleManager.java
+++ b/incubator/net/src/main/java/org/onosproject/incubator/net/virtual/impl/VirtualNetworkFlowRuleManager.java
@@ -49,7 +49,9 @@
 import org.onosproject.net.flow.FlowRuleOperation;
 import org.onosproject.net.flow.FlowRuleOperations;
 import org.onosproject.net.flow.FlowRuleService;
+import org.onosproject.net.flow.FlowRuleStoreDelegate;
 import org.onosproject.net.flow.TableStatisticsEntry;
+import org.onosproject.net.provider.ProviderId;
 import org.slf4j.Logger;
 import org.slf4j.LoggerFactory;
 
@@ -64,6 +66,8 @@
 
 import static com.google.common.base.Preconditions.checkNotNull;
 import static org.onlab.util.Tools.groupedThreads;
+import static org.onosproject.net.flow.FlowRuleEvent.Type.RULE_ADD_REQUESTED;
+import static org.onosproject.net.flow.FlowRuleEvent.Type.RULE_REMOVE_REQUESTED;
 
 /**
  * Flow rule service implementation built on the virtual network service.
@@ -98,6 +102,8 @@
     private VirtualProviderRegistryService providerRegistryService = null;
     private InternalFlowRuleProviderService innerProviderService = null;
 
+    private final FlowRuleStoreDelegate storeDelegate;
+
     /**
      * Creates a new VirtualNetworkFlowRuleService object.
      *
@@ -109,6 +115,7 @@
         super(virtualNetworkManager, networkId);
 
         store = serviceDirectory.get(VirtualNetworkFlowRuleStore.class);
+
         idGenerator = serviceDirectory.get(CoreService.class)
                 .getIdGenerator(VIRTUAL_FLOW_OP_TOPIC + networkId().toString());
         providerRegistryService =
@@ -116,7 +123,9 @@
         innerProviderService = new InternalFlowRuleProviderService();
         providerRegistryService.registerProviderService(networkId(), innerProviderService);
 
-        this.deviceService = manager.get(networkId(), DeviceService.class);
+        this.deviceService = manager.get(networkId, DeviceService.class);
+        this.storeDelegate = new InternalStoreDelegate();
+        store.setDelegate(networkId, this.storeDelegate);
     }
 
     @Override
@@ -288,13 +297,23 @@
         }
     }
 
-    private class InternalFlowRuleProviderService
+    private final class InternalFlowRuleProviderService
             extends AbstractVirtualProviderService<VirtualFlowRuleProvider>
             implements VirtualFlowRuleProviderService {
 
         final Map<FlowEntry, Long> firstSeen = Maps.newConcurrentMap();
         final Map<FlowEntry, Long> lastSeen = Maps.newConcurrentMap();
 
+        private InternalFlowRuleProviderService() {
+            //TODO: find a proper virtual provider.
+            Set<ProviderId> providerIds =
+                    providerRegistryService.getProvidersByService(this);
+            ProviderId providerId = providerIds.stream().findFirst().get();
+            VirtualFlowRuleProvider provider = (VirtualFlowRuleProvider)
+                    providerRegistryService.getProvider(providerId);
+            setProvider(provider);
+        }
+
         @Override
         public void flowRemoved(FlowEntry flowEntry) {
             checkNotNull(flowEntry, FLOW_RULE_NULL);
@@ -310,7 +329,6 @@
             if (flowEntry.reason() == FlowEntry.FlowRemoveReason.HARD_TIMEOUT) {
                 ((DefaultFlowEntry) stored).setState(FlowEntry.FlowEntryState.REMOVED);
             }
-            Device device = deviceService.getDevice(flowEntry.deviceId());
 
             //FIXME: obtains provider from devices providerId()
             FlowRuleEvent event = null;
@@ -333,7 +351,6 @@
             }
         }
 
-
         private void flowMissing(FlowEntry flowRule) {
             checkNotNull(flowRule, FLOW_RULE_NULL);
             checkValidity();
@@ -347,6 +364,7 @@
                 case ADDED:
                 case PENDING_ADD:
                     event = store.pendingFlowRule(networkId(), flowRule);
+
                     try {
                         provider().applyFlowRule(networkId(), flowRule);
                     } catch (UnsupportedOperationException e) {
@@ -494,4 +512,53 @@
             store.updateTableStatistics(networkId(), deviceId, tableStats);
         }
     }
+
+    // Store delegate to re-post events emitted from the store.
+    private class InternalStoreDelegate implements FlowRuleStoreDelegate {
+
+        // TODO: Right now we only dispatch events at individual flowEntry level.
+        // It may be more efficient for also dispatch events as a batch.
+        @Override
+        public void notify(FlowRuleBatchEvent event) {
+            final FlowRuleBatchRequest request = event.subject();
+            switch (event.type()) {
+                case BATCH_OPERATION_REQUESTED:
+                    // Request has been forwarded to MASTER Node, and was
+                    request.ops().forEach(
+                            op -> {
+                                switch (op.operator()) {
+                                    case ADD:
+                                        post(new FlowRuleEvent(RULE_ADD_REQUESTED, op.target()));
+                                        break;
+                                    case REMOVE:
+                                        post(new FlowRuleEvent(RULE_REMOVE_REQUESTED, op.target()));
+                                        break;
+                                    case MODIFY:
+                                        //TODO: do something here when the time comes.
+                                        break;
+                                    default:
+                                        log.warn("Unknown flow operation operator: {}", op.operator());
+                                }
+                            }
+                    );
+
+                    DeviceId deviceId = event.deviceId();
+                    FlowRuleBatchOperation batchOperation = request.asBatchOperation(deviceId);
+
+                    VirtualFlowRuleProvider provider = innerProviderService.provider();
+                    if (provider != null) {
+                        provider.executeBatch(networkId, batchOperation);
+                    }
+
+                    break;
+
+                case BATCH_OPERATION_COMPLETED:
+                    //TODO: do post-processing for batch operations.
+                    break;
+
+                default:
+                    break;
+            }
+        }
+    }
 }
diff --git a/incubator/net/src/main/java/org/onosproject/incubator/net/virtual/impl/provider/VirtualProviderManager.java b/incubator/net/src/main/java/org/onosproject/incubator/net/virtual/impl/provider/VirtualProviderManager.java
index 59c6630..40a791f 100644
--- a/incubator/net/src/main/java/org/onosproject/incubator/net/virtual/impl/provider/VirtualProviderManager.java
+++ b/incubator/net/src/main/java/org/onosproject/incubator/net/virtual/impl/provider/VirtualProviderManager.java
@@ -110,13 +110,14 @@
     }
 
     @Override
-    public Set<ProviderId> getProvidersByService(VirtualProviderService virtualProviderService) {
+    public Set<ProviderId> getProvidersByService(VirtualProviderService
+                                                             virtualProviderService) {
         Class clazz = getProviderClass(virtualProviderService);
 
         return ImmutableSet.copyOf(providers.values().stream()
-                                           .filter(p -> p.getClass()
-                                                   .isAssignableFrom(clazz))
-                                           .map(p -> p.id()).collect(Collectors.toSet()));
+                                           .filter(clazz::isInstance)
+                                           .map(VirtualProvider::id)
+                                           .collect(Collectors.toSet()));
     }
 
     @Override
@@ -144,8 +145,7 @@
         }
 
         return services.stream()
-                .filter(s -> virtualProvider.getClass()
-                        .isAssignableFrom(getProviderClass(s)))
+                .filter(s -> getProviderClass(s).isInstance(virtualProvider))
                 .findFirst().get();
     }
 
diff --git a/incubator/net/src/test/java/org/onosproject/incubator/net/virtual/impl/VirtualNetworkFlowRuleManagerTest.java b/incubator/net/src/test/java/org/onosproject/incubator/net/virtual/impl/VirtualNetworkFlowRuleManagerTest.java
index 593c6d6..40658bd 100644
--- a/incubator/net/src/test/java/org/onosproject/incubator/net/virtual/impl/VirtualNetworkFlowRuleManagerTest.java
+++ b/incubator/net/src/test/java/org/onosproject/incubator/net/virtual/impl/VirtualNetworkFlowRuleManagerTest.java
@@ -16,6 +16,8 @@
 
 package org.onosproject.incubator.net.virtual.impl;
 
+import com.google.common.collect.ImmutableList;
+import com.google.common.collect.ImmutableMap;
 import com.google.common.collect.Lists;
 import com.google.common.collect.Sets;
 import com.google.common.util.concurrent.MoreExecutors;
@@ -29,6 +31,7 @@
 import org.onosproject.common.event.impl.TestEventDispatcher;
 import org.onosproject.core.ApplicationId;
 import org.onosproject.core.CoreService;
+import org.onosproject.event.EventDeliveryService;
 import org.onosproject.incubator.net.virtual.NetworkId;
 import org.onosproject.incubator.net.virtual.TenantId;
 import org.onosproject.incubator.net.virtual.VirtualDevice;
@@ -36,11 +39,15 @@
 import org.onosproject.incubator.net.virtual.VirtualNetwork;
 import org.onosproject.incubator.net.virtual.VirtualNetworkFlowRuleStore;
 import org.onosproject.incubator.net.virtual.VirtualNetworkStore;
+import org.onosproject.incubator.net.virtual.event.VirtualEvent;
+import org.onosproject.incubator.net.virtual.event.VirtualListenerRegistryManager;
 import org.onosproject.incubator.net.virtual.impl.provider.VirtualProviderManager;
 import org.onosproject.incubator.net.virtual.provider.AbstractVirtualProvider;
 import org.onosproject.incubator.net.virtual.provider.VirtualFlowRuleProvider;
+import org.onosproject.incubator.net.virtual.provider.VirtualFlowRuleProviderService;
 import org.onosproject.incubator.net.virtual.provider.VirtualProviderRegistryService;
 import org.onosproject.incubator.store.virtual.impl.DistributedVirtualNetworkStore;
+import org.onosproject.incubator.store.virtual.impl.SimpleVirtualFlowRuleStore;
 import org.onosproject.net.ConnectPoint;
 import org.onosproject.net.DeviceId;
 import org.onosproject.net.Link;
@@ -50,17 +57,12 @@
 import org.onosproject.net.flow.DefaultFlowEntry;
 import org.onosproject.net.flow.DefaultFlowRule;
 import org.onosproject.net.flow.FlowEntry;
-import org.onosproject.net.flow.FlowId;
 import org.onosproject.net.flow.FlowRule;
-import org.onosproject.net.flow.FlowRuleBatchEntry;
-import org.onosproject.net.flow.FlowRuleBatchEvent;
 import org.onosproject.net.flow.FlowRuleBatchOperation;
 import org.onosproject.net.flow.FlowRuleEvent;
 import org.onosproject.net.flow.FlowRuleListener;
 import org.onosproject.net.flow.FlowRuleService;
-import org.onosproject.net.flow.FlowRuleStoreDelegate;
 import org.onosproject.net.flow.StoredFlowEntry;
-import org.onosproject.net.flow.TableStatisticsEntry;
 import org.onosproject.net.flow.TrafficSelector;
 import org.onosproject.net.flow.TrafficTreatment;
 import org.onosproject.net.flow.criteria.Criterion;
@@ -73,13 +75,13 @@
 
 import java.util.ArrayList;
 import java.util.Collections;
-import java.util.HashSet;
+import java.util.HashMap;
 import java.util.List;
+import java.util.Map;
 import java.util.Set;
-import java.util.concurrent.ConcurrentHashMap;
-import java.util.concurrent.ConcurrentMap;
 
 import static org.junit.Assert.*;
+import static org.onosproject.net.flow.FlowRuleEvent.Type.*;
 
 public class VirtualNetworkFlowRuleManagerTest extends TestDeviceParams {
     private static final int TIMEOUT = 10;
@@ -89,12 +91,18 @@
     private TestableIntentService intentService = new FakeIntentManager();
     private ServiceDirectory testDirectory;
     private VirtualNetworkFlowRuleStore flowRuleStore;
-    private VirtualProviderRegistryService providerRegistryService;
+    private VirtualProviderManager providerRegistryService;
+
+    private EventDeliveryService eventDeliveryService;
+    VirtualListenerRegistryManager listenerRegistryManager =
+            VirtualListenerRegistryManager.getInstance();
 
     private VirtualNetworkFlowRuleManager vnetFlowRuleService1;
     private VirtualNetworkFlowRuleManager vnetFlowRuleService2;
 
     private VirtualFlowRuleProvider provider = new TestProvider();
+    private VirtualFlowRuleProviderService providerService1;
+    private VirtualFlowRuleProviderService providerService2;
 
     protected TestFlowRuleListener listener1 = new TestFlowRuleListener();
     protected TestFlowRuleListener listener2 = new TestFlowRuleListener();
@@ -116,7 +124,7 @@
         TestUtils.setField(virtualNetworkManagerStore, "storageService", new TestStorageService());
         virtualNetworkManagerStore.activate();
 
-        flowRuleStore = new TestVirtualFlowRuleStore();
+        flowRuleStore = new SimpleVirtualFlowRuleStore();
 
         providerRegistryService = new VirtualProviderManager();
         providerRegistryService.registerProvider(provider);
@@ -125,7 +133,10 @@
         manager.store = virtualNetworkManagerStore;
         manager.intentService = intentService;
         TestUtils.setField(manager, "coreService", coreService);
-        NetTestTools.injectEventDispatcher(manager, new TestEventDispatcher());
+
+        eventDeliveryService = new TestEventDispatcher();
+        NetTestTools.injectEventDispatcher(manager, eventDeliveryService);
+        eventDeliveryService.addSink(VirtualEvent.class, listenerRegistryManager);
 
         appId = new TestApplicationId("FlowRuleManagerTest");
 
@@ -133,6 +144,7 @@
                 .add(VirtualNetworkStore.class, virtualNetworkManagerStore)
                 .add(CoreService.class, coreService)
                 .add(VirtualProviderRegistryService.class, providerRegistryService)
+                .add(EventDeliveryService.class, eventDeliveryService)
                 .add(VirtualNetworkFlowRuleStore.class, flowRuleStore);
         TestUtils.setField(manager, "serviceDirectory", testDirectory);
 
@@ -144,11 +156,17 @@
         vnetFlowRuleService1 = new VirtualNetworkFlowRuleManager(manager, vnet1.id());
         vnetFlowRuleService2 = new VirtualNetworkFlowRuleManager(manager, vnet2.id());
         vnetFlowRuleService1.addListener(listener1);
+        vnetFlowRuleService2.addListener(listener2);
 
         vnetFlowRuleService1.operationsService = MoreExecutors.newDirectExecutorService();
         vnetFlowRuleService2.operationsService = MoreExecutors.newDirectExecutorService();
         vnetFlowRuleService1.deviceInstallers = MoreExecutors.newDirectExecutorService();
         vnetFlowRuleService2.deviceInstallers = MoreExecutors.newDirectExecutorService();
+
+        providerService1 = (VirtualFlowRuleProviderService)
+                providerRegistryService.getProviderService(vnet1.id(), provider);
+        providerService2 = (VirtualFlowRuleProviderService)
+                providerRegistryService.getProviderService(vnet2.id(), provider);
     }
 
     @After
@@ -246,6 +264,7 @@
                    Sets.newHashSet(vnetFlowRuleService1.getFlowEntries(DID1)).isEmpty());
         assertTrue("store should be empty",
                    Sets.newHashSet(vnetFlowRuleService2.getFlowEntries(DID1)).isEmpty());
+
         FlowRule f1 = addFlowRule(1);
         FlowRule f2 = addFlowRule(2);
 
@@ -254,6 +273,180 @@
 
         assertEquals("2 rules should exist", 2, flowCount(vnetFlowRuleService1));
         assertEquals("0 rules should exist", 0, flowCount(vnetFlowRuleService2));
+
+        providerService1.pushFlowMetrics(DID1, ImmutableList.of(fe1, fe2));
+        validateEvents(listener1, RULE_ADD_REQUESTED, RULE_ADD_REQUESTED,
+                       RULE_ADDED, RULE_ADDED);
+
+        addFlowRule(1);
+        assertEquals("should still be 2 rules", 2, flowCount(vnetFlowRuleService1));
+        System.err.println("events :" + listener1.events);
+        assertEquals("0 rules should exist", 0, flowCount(vnetFlowRuleService2));
+
+        providerService1.pushFlowMetrics(DID1, ImmutableList.of(fe1));
+        validateEvents(listener1, RULE_UPDATED, RULE_UPDATED);
+    }
+
+    @Test
+    public void applyFlowRules() {
+        FlowRule r1 = flowRule(1, 1);
+        FlowRule r2 = flowRule(2, 2);
+        FlowRule r3 = flowRule(3, 3);
+
+        assertTrue("store should be empty",
+                   Sets.newHashSet(vnetFlowRuleService1.getFlowEntries(DID1)).isEmpty());
+        vnetFlowRuleService1.applyFlowRules(r1, r2, r3);
+        assertEquals("3 rules should exist", 3, flowCount(vnetFlowRuleService1));
+        assertTrue("Entries should be pending add.",
+                   validateState(ImmutableMap.of(
+                           r1, FlowEntry.FlowEntryState.PENDING_ADD,
+                           r2, FlowEntry.FlowEntryState.PENDING_ADD,
+                           r3, FlowEntry.FlowEntryState.PENDING_ADD)));
+    }
+
+    @Test
+    public void purgeFlowRules() {
+        FlowRule f1 = addFlowRule(1);
+        FlowRule f2 = addFlowRule(2);
+        FlowRule f3 = addFlowRule(3);
+        assertEquals("3 rules should exist", 3, flowCount(vnetFlowRuleService1));
+        FlowEntry fe1 = new DefaultFlowEntry(f1);
+        FlowEntry fe2 = new DefaultFlowEntry(f2);
+        FlowEntry fe3 = new DefaultFlowEntry(f3);
+        providerService1.pushFlowMetrics(DID1, ImmutableList.of(fe1, fe2, fe3));
+        validateEvents(listener1, RULE_ADD_REQUESTED, RULE_ADD_REQUESTED, RULE_ADD_REQUESTED,
+                       RULE_ADDED, RULE_ADDED, RULE_ADDED);
+        vnetFlowRuleService1.purgeFlowRules(DID1);
+        assertEquals("0 rule should exist", 0, flowCount(vnetFlowRuleService1));
+    }
+
+    @Test
+    public void removeFlowRules() {
+        FlowRule f1 = addFlowRule(1);
+        FlowRule f2 = addFlowRule(2);
+        FlowRule f3 = addFlowRule(3);
+        assertEquals("3 rules should exist", 3, flowCount(vnetFlowRuleService1));
+
+        FlowEntry fe1 = new DefaultFlowEntry(f1);
+        FlowEntry fe2 = new DefaultFlowEntry(f2);
+        FlowEntry fe3 = new DefaultFlowEntry(f3);
+        providerService1.pushFlowMetrics(DID1, ImmutableList.of(fe1, fe2, fe3));
+        validateEvents(listener1, RULE_ADD_REQUESTED, RULE_ADD_REQUESTED, RULE_ADD_REQUESTED,
+                       RULE_ADDED, RULE_ADDED, RULE_ADDED);
+
+        vnetFlowRuleService1.removeFlowRules(f1, f2);
+        //removing from north, so no events generated
+        validateEvents(listener1, RULE_REMOVE_REQUESTED, RULE_REMOVE_REQUESTED);
+        assertEquals("3 rule should exist", 3, flowCount(vnetFlowRuleService1));
+        assertTrue("Entries should be pending remove.",
+                   validateState(ImmutableMap.of(
+                           f1, FlowEntry.FlowEntryState.PENDING_REMOVE,
+                           f2, FlowEntry.FlowEntryState.PENDING_REMOVE,
+                           f3, FlowEntry.FlowEntryState.ADDED)));
+
+        vnetFlowRuleService1.removeFlowRules(f1);
+        assertEquals("3 rule should still exist", 3, flowCount(vnetFlowRuleService1));
+    }
+
+    @Test
+    public void flowRemoved() {
+        FlowRule f1 = addFlowRule(1);
+        FlowRule f2 = addFlowRule(2);
+        StoredFlowEntry fe1 = new DefaultFlowEntry(f1);
+        FlowEntry fe2 = new DefaultFlowEntry(f2);
+
+        providerService1.pushFlowMetrics(DID1, ImmutableList.of(fe1, fe2));
+        vnetFlowRuleService1.removeFlowRules(f1);
+
+        //FIXME modification of "stored" flow entry outside of store
+        fe1.setState(FlowEntry.FlowEntryState.REMOVED);
+
+        providerService1.flowRemoved(fe1);
+
+        validateEvents(listener1, RULE_ADD_REQUESTED, RULE_ADD_REQUESTED, RULE_ADDED,
+                       RULE_ADDED, RULE_REMOVE_REQUESTED, RULE_REMOVED);
+
+        providerService1.flowRemoved(fe1);
+        validateEvents(listener1);
+
+        FlowRule f3 = flowRule(3, 3);
+        FlowEntry fe3 = new DefaultFlowEntry(f3);
+        vnetFlowRuleService1.applyFlowRules(f3);
+
+        providerService1.pushFlowMetrics(DID1, Collections.singletonList(fe3));
+        validateEvents(listener1, RULE_ADD_REQUESTED, RULE_ADDED, RULE_UPDATED);
+
+        providerService1.flowRemoved(fe3);
+        validateEvents(listener1);
+    }
+
+    @Test
+    public void extraneousFlow() {
+        FlowRule f1 = flowRule(1, 1);
+        FlowRule f2 = flowRule(2, 2);
+        FlowRule f3 = flowRule(3, 3);
+        vnetFlowRuleService1.applyFlowRules(f1, f2);
+
+        FlowEntry fe1 = new DefaultFlowEntry(f1);
+        FlowEntry fe2 = new DefaultFlowEntry(f2);
+        FlowEntry fe3 = new DefaultFlowEntry(f3);
+
+
+        providerService1.pushFlowMetrics(DID1, Lists.newArrayList(fe1, fe2, fe3));
+
+        validateEvents(listener1, RULE_ADD_REQUESTED, RULE_ADD_REQUESTED,
+                       RULE_ADDED, RULE_ADDED);
+    }
+
+    /*
+     * Tests whether a rule that was marked for removal but no flowRemoved was received
+     * is indeed removed at the next stats update.
+     */
+    @Test
+    public void flowMissingRemove() {
+        FlowRule f1 = flowRule(1, 1);
+        FlowRule f2 = flowRule(2, 2);
+        FlowRule f3 = flowRule(3, 3);
+
+        FlowEntry fe1 = new DefaultFlowEntry(f1);
+        FlowEntry fe2 = new DefaultFlowEntry(f2);
+        vnetFlowRuleService1.applyFlowRules(f1, f2, f3);
+
+        vnetFlowRuleService1.removeFlowRules(f3);
+
+        providerService1.pushFlowMetrics(DID1, Lists.newArrayList(fe1, fe2));
+
+        validateEvents(listener1, RULE_ADD_REQUESTED, RULE_ADD_REQUESTED, RULE_ADD_REQUESTED,
+                       RULE_REMOVE_REQUESTED, RULE_ADDED, RULE_ADDED, RULE_REMOVED);
+    }
+
+    @Test
+    public void removeByAppId() {
+        FlowRule f1 = flowRule(1, 1);
+        FlowRule f2 = flowRule(2, 2);
+        vnetFlowRuleService1.applyFlowRules(f1, f2);
+
+        vnetFlowRuleService1.removeFlowRulesById(appId);
+
+        //only check that we are in pending remove. Events and actual remove state will
+        // be set by flowRemoved call.
+        validateState(ImmutableMap.of(
+                f1, FlowEntry.FlowEntryState.PENDING_REMOVE,
+                f2, FlowEntry.FlowEntryState.PENDING_REMOVE));
+    }
+
+    //TODO:Tests for fallback
+
+    private boolean validateState(Map<FlowRule, FlowEntry.FlowEntryState> expected) {
+        Map<FlowRule, FlowEntry.FlowEntryState> expectedToCheck = new HashMap<>(expected);
+        Iterable<FlowEntry> rules = vnetFlowRuleService1.getFlowEntries(DID1);
+        for (FlowEntry f : rules) {
+            assertTrue("Unexpected FlowRule " + f, expectedToCheck.containsKey(f));
+            assertEquals("FlowEntry" + f, expectedToCheck.get(f), f.state());
+            expectedToCheck.remove(f);
+        }
+        assertEquals(Collections.emptySet(), expectedToCheck.entrySet());
+        return true;
     }
 
     private class TestSelector implements TrafficSelector {
@@ -288,7 +481,6 @@
             }
             return false;
         }
-
     }
 
     private class TestTreatment implements TrafficTreatment {
@@ -349,118 +541,6 @@
         }
     }
 
-    private class TestVirtualFlowRuleStore implements VirtualNetworkFlowRuleStore {
-
-        private final ConcurrentMap<NetworkId,
-                ConcurrentMap<DeviceId, ConcurrentMap<FlowId, List<StoredFlowEntry>>>>
-                flowEntries = new ConcurrentHashMap<>();
-
-        @Override
-        public void setDelegate(FlowRuleStoreDelegate delegate) {
-
-        }
-
-        @Override
-        public void unsetDelegate(FlowRuleStoreDelegate delegate) {
-
-        }
-
-        @Override
-        public boolean hasDelegate() {
-            return false;
-        }
-
-        @Override
-        public int getFlowRuleCount(NetworkId networkId) {
-            return 0;
-        }
-
-        @Override
-        public FlowEntry getFlowEntry(NetworkId networkId, FlowRule rule) {
-            return null;
-        }
-
-        @Override
-        public Iterable<FlowEntry> getFlowEntries(NetworkId networkId, DeviceId deviceId) {
-            HashSet<FlowEntry> entries = Sets.newHashSet();
-
-            if (flowEntries.get(networkId) == null
-                    || flowEntries.get(networkId).get(deviceId) == null) {
-                return entries;
-            }
-
-            flowEntries.get(networkId).get(deviceId).values().forEach(e -> entries.addAll(e));
-
-            return entries;
-        }
-
-        @Override
-        public void storeFlowRule(NetworkId networkId, FlowRule rule) {
-            StoredFlowEntry entry = new DefaultFlowEntry(rule);
-            flowEntries.putIfAbsent(networkId, new ConcurrentHashMap<>());
-            flowEntries.get(networkId).putIfAbsent(rule.deviceId(), new ConcurrentHashMap<>());
-            flowEntries.get(networkId).get(rule.deviceId()).putIfAbsent(rule.id(), Lists.newArrayList());
-            flowEntries.get(networkId).get(rule.deviceId()).get(rule.id()).add(entry);
-        }
-
-        @Override
-        public void storeBatch(NetworkId networkId, FlowRuleBatchOperation batchOperation) {
-            for (FlowRuleBatchEntry entry : batchOperation.getOperations()) {
-                final FlowRule flowRule = entry.target();
-                if (entry.operator().equals(FlowRuleBatchEntry.FlowRuleOperation.ADD)) {
-                    storeFlowRule(networkId, flowRule);
-                } else if (entry.operator().equals(FlowRuleBatchEntry.FlowRuleOperation.REMOVE)) {
-                    deleteFlowRule(networkId, flowRule);
-                } else {
-                    throw new UnsupportedOperationException("Unsupported operation type");
-                }
-            }
-        }
-
-        @Override
-        public void batchOperationComplete(NetworkId networkId, FlowRuleBatchEvent event) {
-
-        }
-
-        @Override
-        public void deleteFlowRule(NetworkId networkId, FlowRule rule) {
-
-        }
-
-        @Override
-        public FlowRuleEvent addOrUpdateFlowRule(NetworkId networkId, FlowEntry rule) {
-            return null;
-        }
-
-        @Override
-        public FlowRuleEvent removeFlowRule(NetworkId networkId, FlowEntry rule) {
-            return null;
-        }
-
-        @Override
-        public FlowRuleEvent pendingFlowRule(NetworkId networkId, FlowEntry rule) {
-            return null;
-        }
-
-        @Override
-        public void purgeFlowRules(NetworkId networkId) {
-
-        }
-
-        @Override
-        public FlowRuleEvent updateTableStatistics(NetworkId networkId,
-                                                   DeviceId deviceId,
-                                                   List<TableStatisticsEntry> tableStats) {
-            return null;
-        }
-
-        @Override
-        public Iterable<TableStatisticsEntry>
-        getTableStatistics(NetworkId networkId, DeviceId deviceId) {
-            return null;
-        }
-    }
-
     private void validateEvents(TestFlowRuleListener listener, FlowRuleEvent.Type... events) {
         if (events == null) {
             assertTrue("events generated", listener.events.isEmpty());
@@ -485,7 +565,7 @@
 
         @Override
         public void event(FlowRuleEvent event) {
-           events.add(event);
+            events.add(event);
         }
     }
 
diff --git a/incubator/store/src/main/java/org/onosproject/incubator/store/virtual/impl/AbstractVirtualStore.java b/incubator/store/src/main/java/org/onosproject/incubator/store/virtual/impl/AbstractVirtualStore.java
new file mode 100644
index 0000000..eab35b1
--- /dev/null
+++ b/incubator/store/src/main/java/org/onosproject/incubator/store/virtual/impl/AbstractVirtualStore.java
@@ -0,0 +1,82 @@
+/*
+ * Copyright 2016-present Open Networking Laboratory
+ *
+ * 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.incubator.store.virtual.impl;
+
+import com.google.common.collect.Maps;
+import org.onosproject.event.Event;
+import org.onosproject.incubator.net.virtual.NetworkId;
+import org.onosproject.incubator.net.virtual.VirtualStore;
+import org.onosproject.store.StoreDelegate;
+
+import java.util.List;
+import java.util.Map;
+
+import static com.google.common.base.Preconditions.checkState;
+
+/**
+ * Base implementation of a virtual store.
+ */
+public class AbstractVirtualStore<E extends Event, D extends StoreDelegate<E>>
+        implements VirtualStore<E, D> {
+
+    protected Map<NetworkId, D> delegateMap = Maps.newConcurrentMap();
+
+    @Override
+    public void setDelegate(NetworkId networkId, D delegate) {
+        checkState(delegateMap.get(networkId) == null
+                           || delegateMap.get(networkId) == delegate,
+                   "Store delegate already set");
+
+        delegateMap.putIfAbsent(networkId, delegate);
+    }
+
+    @Override
+    public void unsetDelegate(NetworkId networkId, D delegate) {
+        if (delegateMap.get(networkId) == delegate) {
+            delegateMap.remove(networkId, delegate);
+        }
+    }
+
+    @Override
+    public boolean hasDelegate(NetworkId networkId) {
+        return delegateMap.get(networkId) != null;
+    }
+
+    /**
+     * Notifies the delegate with the specified event.
+     *
+     * @param networkId a virtual network identifier
+     * @param event event to delegate
+     */
+    protected void notifyDelegate(NetworkId networkId, E event) {
+        if (delegateMap.get(networkId) != null) {
+            delegateMap.get(networkId).notify(event);
+        }
+    }
+
+    /**
+     * Notifies the delegate with the specified list of events.
+     *
+     * @param networkId a virtual network identifier
+     * @param events list of events to delegate
+     */
+    protected void notifyDelegate(NetworkId networkId, List<E> events) {
+        for (E event: events) {
+            notifyDelegate(networkId, event);
+        }
+    }
+}
diff --git a/incubator/store/src/main/java/org/onosproject/incubator/store/virtual/impl/SimpleVirtualFlowRuleStore.java b/incubator/store/src/main/java/org/onosproject/incubator/store/virtual/impl/SimpleVirtualFlowRuleStore.java
new file mode 100644
index 0000000..ff13ac2
--- /dev/null
+++ b/incubator/store/src/main/java/org/onosproject/incubator/store/virtual/impl/SimpleVirtualFlowRuleStore.java
@@ -0,0 +1,403 @@
+/*
+ * Copyright 2016-present Open Networking Laboratory
+ *
+ * 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.incubator.store.virtual.impl;
+
+import com.google.common.cache.Cache;
+import com.google.common.cache.CacheBuilder;
+import com.google.common.cache.RemovalListener;
+import com.google.common.cache.RemovalNotification;
+import com.google.common.collect.FluentIterable;
+import com.google.common.collect.Sets;
+import com.google.common.util.concurrent.SettableFuture;
+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.Modified;
+import org.apache.felix.scr.annotations.Property;
+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.incubator.net.virtual.NetworkId;
+import org.onosproject.incubator.net.virtual.VirtualNetworkFlowRuleStore;
+import org.onosproject.net.DeviceId;
+import org.onosproject.net.flow.CompletedBatchOperation;
+import org.onosproject.net.flow.DefaultFlowEntry;
+import org.onosproject.net.flow.FlowEntry;
+import org.onosproject.net.flow.FlowId;
+import org.onosproject.net.flow.FlowRule;
+import org.onosproject.net.flow.FlowRuleBatchEntry;
+import org.onosproject.net.flow.FlowRuleBatchEvent;
+import org.onosproject.net.flow.FlowRuleBatchOperation;
+import org.onosproject.net.flow.FlowRuleBatchRequest;
+import org.onosproject.net.flow.FlowRuleEvent;
+import org.onosproject.net.flow.FlowRuleStoreDelegate;
+import org.onosproject.net.flow.StoredFlowEntry;
+import org.onosproject.net.flow.TableStatisticsEntry;
+import org.onosproject.store.service.StorageService;
+import org.osgi.service.component.ComponentContext;
+import org.slf4j.Logger;
+
+import java.util.ArrayList;
+import java.util.Collections;
+import java.util.Dictionary;
+import java.util.List;
+import java.util.concurrent.ConcurrentHashMap;
+import java.util.concurrent.ConcurrentMap;
+import java.util.concurrent.CopyOnWriteArrayList;
+import java.util.concurrent.ExecutionException;
+import java.util.concurrent.TimeUnit;
+import java.util.concurrent.TimeoutException;
+import java.util.concurrent.atomic.AtomicInteger;
+
+import static org.onosproject.net.flow.FlowRuleEvent.Type.RULE_REMOVED;
+import static org.slf4j.LoggerFactory.getLogger;
+
+//TODO: support distributed flowrule store for virtual networks
+
+@Component(immediate = true)
+@Service
+public class SimpleVirtualFlowRuleStore
+        extends AbstractVirtualStore<FlowRuleBatchEvent, FlowRuleStoreDelegate>
+        implements VirtualNetworkFlowRuleStore {
+
+    private final Logger log = getLogger(getClass());
+
+    @Reference(cardinality = ReferenceCardinality.MANDATORY_UNARY)
+    protected StorageService storageService;
+
+    private final ConcurrentMap<NetworkId,
+            ConcurrentMap<DeviceId, ConcurrentMap<FlowId, List<StoredFlowEntry>>>>
+            flowEntries = new ConcurrentHashMap<>();
+
+
+    private final AtomicInteger localBatchIdGen = new AtomicInteger();
+
+    private static final int DEFAULT_PENDING_FUTURE_TIMEOUT_MINUTES = 5;
+    @Property(name = "pendingFutureTimeoutMinutes", intValue = DEFAULT_PENDING_FUTURE_TIMEOUT_MINUTES,
+            label = "Expiration time after an entry is created that it should be automatically removed")
+    private int pendingFutureTimeoutMinutes = DEFAULT_PENDING_FUTURE_TIMEOUT_MINUTES;
+
+    private Cache<Integer, SettableFuture<CompletedBatchOperation>> pendingFutures =
+            CacheBuilder.newBuilder()
+                    .expireAfterWrite(pendingFutureTimeoutMinutes, TimeUnit.MINUTES)
+                    .removalListener(new TimeoutFuture())
+                    .build();
+
+    @Activate
+    public void activate() {
+        log.info("Started");
+    }
+
+    @Deactivate
+    public void deactivate() {
+        flowEntries.clear();
+        log.info("Stopped");
+    }
+
+    @Modified
+    public void modified(ComponentContext context) {
+
+        readComponentConfiguration(context);
+
+        // Reset Cache and copy all.
+        Cache<Integer, SettableFuture<CompletedBatchOperation>> prevFutures = pendingFutures;
+        pendingFutures = CacheBuilder.newBuilder()
+                .expireAfterWrite(pendingFutureTimeoutMinutes, TimeUnit.MINUTES)
+                .removalListener(new TimeoutFuture())
+                .build();
+
+        pendingFutures.putAll(prevFutures.asMap());
+    }
+
+    /**
+     * Extracts properties from the component configuration context.
+     *
+     * @param context the component context
+     */
+    private void readComponentConfiguration(ComponentContext context) {
+        Dictionary<?, ?> properties = context.getProperties();
+
+        Integer newPendingFutureTimeoutMinutes =
+                Tools.getIntegerProperty(properties, "pendingFutureTimeoutMinutes");
+        if (newPendingFutureTimeoutMinutes == null) {
+            pendingFutureTimeoutMinutes = DEFAULT_PENDING_FUTURE_TIMEOUT_MINUTES;
+            log.info("Pending future timeout is not configured, " +
+                             "using current value of {}", pendingFutureTimeoutMinutes);
+        } else {
+            pendingFutureTimeoutMinutes = newPendingFutureTimeoutMinutes;
+            log.info("Configured. Pending future timeout is configured to {}",
+                     pendingFutureTimeoutMinutes);
+        }
+    }
+
+    @Override
+    public int getFlowRuleCount(NetworkId networkId) {
+
+        int sum = 0;
+        for (ConcurrentMap<FlowId, List<StoredFlowEntry>> ft :
+                flowEntries.get(networkId).values()) {
+            for (List<StoredFlowEntry> fes : ft.values()) {
+                sum += fes.size();
+            }
+        }
+        return sum;
+    }
+
+    @Override
+    public FlowEntry getFlowEntry(NetworkId networkId, FlowRule rule) {
+        return getFlowEntryInternal(networkId, rule.deviceId(), rule);
+    }
+
+    @Override
+    public Iterable<FlowEntry> getFlowEntries(NetworkId networkId, DeviceId deviceId) {
+        return FluentIterable.from(getFlowTable(networkId, deviceId).values())
+                .transformAndConcat(Collections::unmodifiableList);
+    }
+
+    @Override
+    public void storeFlowRule(NetworkId networkId, FlowRule rule) {
+        storeFlowRuleInternal(networkId, rule);
+    }
+
+    @Override
+    public void storeBatch(NetworkId networkId, FlowRuleBatchOperation batchOperation) {
+        List<FlowRuleBatchEntry> toAdd = new ArrayList<>();
+        List<FlowRuleBatchEntry> toRemove = new ArrayList<>();
+
+        for (FlowRuleBatchEntry entry : batchOperation.getOperations()) {
+            final FlowRule flowRule = entry.target();
+            if (entry.operator().equals(FlowRuleBatchEntry.FlowRuleOperation.ADD)) {
+                if (!getFlowEntries(networkId, flowRule.deviceId(),
+                                    flowRule.id()).contains(flowRule)) {
+                    storeFlowRule(networkId, flowRule);
+                    toAdd.add(entry);
+                }
+            } else if (entry.operator().equals(FlowRuleBatchEntry.FlowRuleOperation.REMOVE)) {
+                if (getFlowEntries(networkId, flowRule.deviceId(), flowRule.id()).contains(flowRule)) {
+                    deleteFlowRule(networkId, flowRule);
+                    toRemove.add(entry);
+                }
+            } else {
+                throw new UnsupportedOperationException("Unsupported operation type");
+            }
+        }
+
+        if (toAdd.isEmpty() && toRemove.isEmpty()) {
+            notifyDelegate(networkId, FlowRuleBatchEvent.completed(
+                    new FlowRuleBatchRequest(batchOperation.id(), Collections.emptySet()),
+                    new CompletedBatchOperation(true, Collections.emptySet(),
+                                                batchOperation.deviceId())));
+            return;
+        }
+
+        SettableFuture<CompletedBatchOperation> r = SettableFuture.create();
+        final int batchId = localBatchIdGen.incrementAndGet();
+
+        pendingFutures.put(batchId, r);
+
+        toAdd.addAll(toRemove);
+        notifyDelegate(networkId, FlowRuleBatchEvent.requested(
+                new FlowRuleBatchRequest(batchId, Sets.newHashSet(toAdd)), batchOperation.deviceId()));
+
+    }
+
+    @Override
+    public void batchOperationComplete(NetworkId networkId, FlowRuleBatchEvent event) {
+        final Long batchId = event.subject().batchId();
+        SettableFuture<CompletedBatchOperation> future
+                = pendingFutures.getIfPresent(batchId);
+        if (future != null) {
+            future.set(event.result());
+            pendingFutures.invalidate(batchId);
+        }
+        notifyDelegate(networkId, event);
+    }
+
+    @Override
+    public void deleteFlowRule(NetworkId networkId, FlowRule rule) {
+        List<StoredFlowEntry> entries = getFlowEntries(networkId, rule.deviceId(), rule.id());
+
+        synchronized (entries) {
+            for (StoredFlowEntry entry : entries) {
+                if (entry.equals(rule)) {
+                    synchronized (entry) {
+                        entry.setState(FlowEntry.FlowEntryState.PENDING_REMOVE);
+                    }
+                }
+            }
+        }
+    }
+
+    @Override
+    public FlowRuleEvent addOrUpdateFlowRule(NetworkId networkId, FlowEntry rule) {
+        // check if this new rule is an update to an existing entry
+        List<StoredFlowEntry> entries = getFlowEntries(networkId, rule.deviceId(), rule.id());
+        synchronized (entries) {
+            for (StoredFlowEntry stored : entries) {
+                if (stored.equals(rule)) {
+                    synchronized (stored) {
+                        //FIXME modification of "stored" flow entry outside of flow table
+                        stored.setBytes(rule.bytes());
+                        stored.setLife(rule.life());
+                        stored.setPackets(rule.packets());
+                        if (stored.state() == FlowEntry.FlowEntryState.PENDING_ADD) {
+                            stored.setState(FlowEntry.FlowEntryState.ADDED);
+                            // TODO: Do we need to change `rule` state?
+                            return new FlowRuleEvent(FlowRuleEvent.Type.RULE_ADDED, rule);
+                        }
+                        return new FlowRuleEvent(FlowRuleEvent.Type.RULE_UPDATED, rule);
+                    }
+                }
+            }
+        }
+
+        // should not reach here
+        // storeFlowRule was expected to be called
+        log.error("FlowRule was not found in store {} to update", rule);
+
+        return null;
+    }
+
+    @Override
+    public FlowRuleEvent removeFlowRule(NetworkId networkId, FlowEntry rule) {
+        // This is where one could mark a rule as removed and still keep it in the store.
+        final DeviceId did = rule.deviceId();
+
+        List<StoredFlowEntry> entries = getFlowEntries(networkId, did, rule.id());
+        synchronized (entries) {
+            if (entries.remove(rule)) {
+                return new FlowRuleEvent(RULE_REMOVED, rule);
+            }
+        }
+        return null;
+    }
+
+    @Override
+    public FlowRuleEvent pendingFlowRule(NetworkId networkId, FlowEntry rule) {
+        List<StoredFlowEntry> entries = getFlowEntries(networkId, rule.deviceId(), rule.id());
+        synchronized (entries) {
+            for (StoredFlowEntry entry : entries) {
+                if (entry.equals(rule) &&
+                        entry.state() != FlowEntry.FlowEntryState.PENDING_ADD) {
+                    synchronized (entry) {
+                        entry.setState(FlowEntry.FlowEntryState.PENDING_ADD);
+                        return new FlowRuleEvent(FlowRuleEvent.Type.RULE_UPDATED, rule);
+                    }
+                }
+            }
+        }
+        return null;
+    }
+
+    @Override
+    public void purgeFlowRule(NetworkId networkId, DeviceId deviceId) {
+        flowEntries.get(networkId).remove(deviceId);
+    }
+
+    @Override
+    public void purgeFlowRules(NetworkId networkId) {
+        flowEntries.get(networkId).clear();
+    }
+
+    @Override
+    public FlowRuleEvent
+    updateTableStatistics(NetworkId networkId, DeviceId deviceId, List<TableStatisticsEntry> tableStats) {
+        //TODO: Table operations are not supported yet
+        return null;
+    }
+
+    @Override
+    public Iterable<TableStatisticsEntry>
+    getTableStatistics(NetworkId networkId, DeviceId deviceId) {
+        //TODO: Table operations are not supported yet
+        return null;
+    }
+
+    /**
+     * Returns the flow table for specified device.
+     *
+     * @param networkId identifier of the virtual network
+     * @param deviceId identifier of the virtual device
+     * @return Map representing Flow Table of given device.
+     */
+    private ConcurrentMap<FlowId, List<StoredFlowEntry>>
+    getFlowTable(NetworkId networkId, DeviceId deviceId) {
+        return flowEntries
+                .computeIfAbsent(networkId, n -> new ConcurrentHashMap<>())
+                .computeIfAbsent(deviceId, k -> new ConcurrentHashMap<>());
+    }
+
+    private List<StoredFlowEntry>
+    getFlowEntries(NetworkId networkId, DeviceId deviceId, FlowId flowId) {
+        final ConcurrentMap<FlowId, List<StoredFlowEntry>> flowTable
+                = getFlowTable(networkId, deviceId);
+
+        List<StoredFlowEntry> r = flowTable.get(flowId);
+        if (r == null) {
+            final List<StoredFlowEntry> concurrentlyAdded;
+            r = new CopyOnWriteArrayList<>();
+            concurrentlyAdded = flowTable.putIfAbsent(flowId, r);
+            if (concurrentlyAdded != null) {
+                return concurrentlyAdded;
+            }
+        }
+        return r;
+    }
+
+    private FlowEntry
+    getFlowEntryInternal(NetworkId networkId, DeviceId deviceId, FlowRule rule) {
+        List<StoredFlowEntry> fes = getFlowEntries(networkId, deviceId, rule.id());
+        for (StoredFlowEntry fe : fes) {
+            if (fe.equals(rule)) {
+                return fe;
+            }
+        }
+        return null;
+    }
+
+    private void storeFlowRuleInternal(NetworkId networkId, FlowRule rule) {
+        StoredFlowEntry f = new DefaultFlowEntry(rule);
+        final DeviceId did = f.deviceId();
+        final FlowId fid = f.id();
+        List<StoredFlowEntry> existing = getFlowEntries(networkId, did, fid);
+        synchronized (existing) {
+            for (StoredFlowEntry fe : existing) {
+                if (fe.equals(rule)) {
+                    // was already there? ignore
+                    return;
+                }
+            }
+            // new flow rule added
+            existing.add(f);
+        }
+    }
+
+    private static final class TimeoutFuture
+            implements RemovalListener<Integer, SettableFuture<CompletedBatchOperation>> {
+        @Override
+        public void onRemoval(RemovalNotification<Integer,
+                SettableFuture<CompletedBatchOperation>> notification) {
+            // wrapping in ExecutionException to support Future.get
+            if (notification.wasEvicted()) {
+                notification.getValue()
+                        .setException(new ExecutionException("Timed out",
+                                                             new TimeoutException()));
+            }
+        }
+    }
+}