CORD-61 Dynamic XConnect support

- Add new XConnectConfig with unit test
- Gather XConnect features into XConnectHandler
- Introduce ObjectiveError.Type.GROUPREMOVALFAILED
- Rename
    - NetworkConfigEventHandler -> AppConfigHandler
    - XConnectNextObjectiveStoreKey -> XConnectStoreKey
    - Test json file
- Refactor

Change-Id: I8ca3176ed976c71ce9e28b7f3722ce80d49c816f
diff --git a/apps/segmentrouting/src/main/java/org/onosproject/segmentrouting/NetworkConfigEventHandler.java b/apps/segmentrouting/src/main/java/org/onosproject/segmentrouting/AppConfigHandler.java
similarity index 90%
rename from apps/segmentrouting/src/main/java/org/onosproject/segmentrouting/NetworkConfigEventHandler.java
rename to apps/segmentrouting/src/main/java/org/onosproject/segmentrouting/AppConfigHandler.java
index d0f6fc4..841ad2f 100644
--- a/apps/segmentrouting/src/main/java/org/onosproject/segmentrouting/NetworkConfigEventHandler.java
+++ b/apps/segmentrouting/src/main/java/org/onosproject/segmentrouting/AppConfigHandler.java
@@ -35,29 +35,29 @@
 import java.util.Set;
 
 /**
- * Handles network config events.
+ * Handles Segment Routing app config events.
  */
-public class NetworkConfigEventHandler {
-    private static final Logger log = LoggerFactory.getLogger(NetworkConfigEventHandler.class);
+public class AppConfigHandler {
+    private static final Logger log = LoggerFactory.getLogger(AppConfigHandler.class);
     private final SegmentRoutingManager srManager;
     private final DeviceService deviceService;
 
     /**
-     * Constructs Network Config Event Handler.
+     * Constructs Segment Routing App Config Handler.
      *
      * @param srManager instance of {@link SegmentRoutingManager}
      */
-    public NetworkConfigEventHandler(SegmentRoutingManager srManager) {
+    public AppConfigHandler(SegmentRoutingManager srManager) {
         this.srManager = srManager;
         this.deviceService = srManager.deviceService;
     }
 
     /**
-     * Processes vRouter config added event.
+     * Processes Segment Routing App Config added event.
      *
      * @param event network config added event
      */
-    protected void processVRouterConfigAdded(NetworkConfigEvent event) {
+    protected void processAppConfigAdded(NetworkConfigEvent event) {
         log.info("Processing vRouter CONFIG_ADDED");
         SegmentRoutingAppConfig config = (SegmentRoutingAppConfig) event.config().get();
         deviceService.getAvailableDevices().forEach(device -> {
@@ -66,11 +66,11 @@
     }
 
     /**
-     * Processes vRouter config updated event.
+     * Processes Segment Routing App Config updated event.
      *
      * @param event network config updated event
      */
-    protected void processVRouterConfigUpdated(NetworkConfigEvent event) {
+    protected void processAppConfigUpdated(NetworkConfigEvent event) {
         log.info("Processing vRouter CONFIG_UPDATED");
         SegmentRoutingAppConfig config = (SegmentRoutingAppConfig) event.config().get();
         SegmentRoutingAppConfig prevConfig = (SegmentRoutingAppConfig) event.prevConfig().get();
@@ -91,11 +91,11 @@
     }
 
     /**
-     * Processes vRouter config removed event.
+     * Processes Segment Routing App Config removed event.
      *
      * @param event network config removed event
      */
-    protected void processVRouterConfigRemoved(NetworkConfigEvent event) {
+    protected void processAppConfigRemoved(NetworkConfigEvent event) {
         log.info("Processing vRouter CONFIG_REMOVED");
         SegmentRoutingAppConfig prevConfig = (SegmentRoutingAppConfig) event.prevConfig().get();
         deviceService.getAvailableDevices().forEach(device -> {
diff --git a/apps/segmentrouting/src/main/java/org/onosproject/segmentrouting/DefaultRoutingHandler.java b/apps/segmentrouting/src/main/java/org/onosproject/segmentrouting/DefaultRoutingHandler.java
index 69bb7bf..358fa75 100644
--- a/apps/segmentrouting/src/main/java/org/onosproject/segmentrouting/DefaultRoutingHandler.java
+++ b/apps/segmentrouting/src/main/java/org/onosproject/segmentrouting/DefaultRoutingHandler.java
@@ -564,7 +564,6 @@
      * @param deviceId Switch ID to set the rules
      */
     public void populatePortAddressingRules(DeviceId deviceId) {
-        rulePopulator.populateXConnectVlanFilters(deviceId);
         rulePopulator.populateRouterIpPunts(deviceId);
 
         // Although device is added, sometimes device store does not have the
diff --git a/apps/segmentrouting/src/main/java/org/onosproject/segmentrouting/McastHandler.java b/apps/segmentrouting/src/main/java/org/onosproject/segmentrouting/McastHandler.java
index b260751..6d5666a 100644
--- a/apps/segmentrouting/src/main/java/org/onosproject/segmentrouting/McastHandler.java
+++ b/apps/segmentrouting/src/main/java/org/onosproject/segmentrouting/McastHandler.java
@@ -79,8 +79,8 @@
     private static final Logger log = LoggerFactory.getLogger(McastHandler.class);
     private final SegmentRoutingManager srManager;
     private final ApplicationId coreAppId;
-    private StorageService storageService;
-    private TopologyService topologyService;
+    private final StorageService storageService;
+    private final TopologyService topologyService;
     private final ConsistentMap<McastStoreKey, NextObjective> mcastNextObjStore;
     private final KryoNamespace.Builder mcastKryo;
     private final ConsistentMap<McastStoreKey, McastRole> mcastRoleStore;
@@ -132,7 +132,7 @@
     /**
      * Read initial multicast from mcast store.
      */
-    public void init() {
+    protected void init() {
         srManager.multicastRouteService.getRoutes().forEach(mcastRoute -> {
             ConnectPoint source = srManager.multicastRouteService.fetchSource(mcastRoute);
             Set<ConnectPoint> sinks = srManager.multicastRouteService.fetchSinks(mcastRoute);
@@ -472,7 +472,7 @@
                             log.warn("Failed to update {} on {}/{}, vlan {}: {}",
                                     mcastIp, deviceId, port.toLong(), assignedVlan, error));
             newNextObj = nextObjBuilder(mcastIp, assignedVlan, existingPorts).add();
-            fwdObj = fwdObjBuilder(mcastIp, assignedVlan, newNextObj.id()).add();
+            fwdObj = fwdObjBuilder(mcastIp, assignedVlan, newNextObj.id()).add(context);
             mcastNextObjStore.put(mcastStoreKey, newNextObj);
             srManager.flowObjectiveService.next(deviceId, newNextObj);
             srManager.flowObjectiveService.forward(deviceId, fwdObj);
@@ -779,11 +779,7 @@
                 // Spine-facing port should have no subnet and no xconnect
                 if (srManager.deviceConfiguration != null &&
                         srManager.deviceConfiguration.getPortSubnet(ingressDevice, port) == null &&
-                        srManager.deviceConfiguration.getXConnects().values().stream()
-                                .allMatch(connectPoints ->
-                                        connectPoints.stream().noneMatch(connectPoint ->
-                                                connectPoint.port().equals(port))
-                                )) {
+                        !srManager.xConnectHandler.hasXConnect(new ConnectPoint(ingressDevice, port))) {
                     return port;
                 }
             }
diff --git a/apps/segmentrouting/src/main/java/org/onosproject/segmentrouting/RoutingRulePopulator.java b/apps/segmentrouting/src/main/java/org/onosproject/segmentrouting/RoutingRulePopulator.java
index 3542ec9..1b4819a 100644
--- a/apps/segmentrouting/src/main/java/org/onosproject/segmentrouting/RoutingRulePopulator.java
+++ b/apps/segmentrouting/src/main/java/org/onosproject/segmentrouting/RoutingRulePopulator.java
@@ -50,7 +50,6 @@
 import java.util.ArrayList;
 import java.util.HashSet;
 import java.util.List;
-import java.util.Map;
 import java.util.Set;
 import java.util.concurrent.atomic.AtomicLong;
 
@@ -694,85 +693,6 @@
         });
     }
 
-    /**
-     * Creates a filtering objective to permit VLAN cross-connect traffic.
-     *
-     * @param deviceId the DPID of the switch
-     */
-    public void populateXConnectVlanFilters(DeviceId deviceId) {
-        Map<VlanId, List<ConnectPoint>> xConnectsForDevice =
-                config.getXConnects();
-        xConnectsForDevice.forEach((vlanId, connectPoints) -> {
-            // Only proceed  the xConnect for given device
-            for (ConnectPoint connectPoint : connectPoints) {
-                if (!connectPoint.deviceId().equals(deviceId)) {
-                    return;
-                }
-            }
-
-            connectPoints.forEach(connectPoint -> {
-                FilteringObjective.Builder fob = DefaultFilteringObjective.builder();
-                fob.withKey(Criteria.matchInPort(connectPoint.port()))
-                        .addCondition(Criteria.matchVlanId(vlanId))
-                        .addCondition(Criteria.matchEthDst(MacAddress.NONE))
-                        .withPriority(SegmentRoutingService.XCONNECT_PRIORITY);
-                fob.permit().fromApp(srManager.appId);
-                ObjectiveContext context = new DefaultObjectiveContext(
-                        (objective) -> log.debug("XConnect filter for {} populated", connectPoint),
-                        (objective, error) ->
-                                log.warn("Failed to populate xconnect filter for {}: {}", connectPoint, error));
-                srManager.flowObjectiveService.filter(deviceId, fob.add(context));
-            });
-        });
-    }
-
-    /**
-     * Populates a forwarding objective that points the VLAN cross-connect
-     * packets to a broadcast group.
-     *
-     * @param deviceId switch ID to set the rules
-     */
-    public void populateXConnectBroadcastRule(DeviceId deviceId) {
-        Map<VlanId, List<ConnectPoint>> xConnects =
-                config.getXConnects();
-        xConnects.forEach((vlanId, connectPoints) -> {
-            // Only proceed  the xConnect for given device
-            for (ConnectPoint connectPoint : connectPoints) {
-                if (!connectPoint.deviceId().equals(deviceId)) {
-                    return;
-                }
-            }
-
-            int nextId = srManager.getXConnectNextObjectiveId(deviceId, vlanId);
-            if (nextId < 0) {
-                log.error("Cannot install cross-connect broadcast rule in dev:{} " +
-                        "due to missing nextId:{}", deviceId, nextId);
-                return;
-            }
-
-            /*
-             * Driver should treat objectives with MacAddress.NONE and !VlanId.NONE
-             * as the VLAN cross-connect broadcast rules
-             */
-            TrafficSelector.Builder sbuilder = DefaultTrafficSelector.builder();
-            sbuilder.matchVlanId(vlanId);
-            sbuilder.matchEthDst(MacAddress.NONE);
-
-            ForwardingObjective.Builder fob = DefaultForwardingObjective.builder();
-            fob.withFlag(Flag.SPECIFIC)
-                    .withSelector(sbuilder.build())
-                    .nextStep(nextId)
-                    .withPriority(SegmentRoutingService.DEFAULT_PRIORITY)
-                    .fromApp(srManager.appId)
-                    .makePermanent();
-            ObjectiveContext context = new DefaultObjectiveContext(
-                    (objective) -> log.debug("XConnect rule for {} populated", xConnects),
-                    (objective, error) ->
-                            log.warn("Failed to populate xconnect rule for {}: {}", xConnects, error));
-            srManager.flowObjectiveService.forward(deviceId, fob.add(context));
-        });
-    }
-
     private int getPriorityFromPrefix(IpPrefix prefix) {
         return (prefix.isIp4()) ?
                 2000 * prefix.prefixLength() + SegmentRoutingService.MIN_IP_PRIORITY :
diff --git a/apps/segmentrouting/src/main/java/org/onosproject/segmentrouting/SegmentRoutingManager.java b/apps/segmentrouting/src/main/java/org/onosproject/segmentrouting/SegmentRoutingManager.java
index a2eea7c..2b87fca 100644
--- a/apps/segmentrouting/src/main/java/org/onosproject/segmentrouting/SegmentRoutingManager.java
+++ b/apps/segmentrouting/src/main/java/org/onosproject/segmentrouting/SegmentRoutingManager.java
@@ -61,6 +61,7 @@
 import org.onosproject.segmentrouting.config.DeviceConfiguration;
 import org.onosproject.segmentrouting.config.SegmentRoutingDeviceConfig;
 import org.onosproject.segmentrouting.config.SegmentRoutingAppConfig;
+import org.onosproject.segmentrouting.config.XConnectConfig;
 import org.onosproject.segmentrouting.grouphandler.DefaultGroupHandler;
 import org.onosproject.segmentrouting.grouphandler.NeighborSet;
 import org.onosproject.segmentrouting.storekey.NeighborSetNextObjectiveStoreKey;
@@ -75,7 +76,7 @@
 import org.onosproject.net.packet.PacketService;
 import org.onosproject.segmentrouting.storekey.SubnetAssignedVidStoreKey;
 import org.onosproject.segmentrouting.storekey.SubnetNextObjectiveStoreKey;
-import org.onosproject.segmentrouting.storekey.XConnectNextObjectiveStoreKey;
+import org.onosproject.segmentrouting.storekey.XConnectStoreKey;
 import org.onosproject.store.serializers.KryoNamespaces;
 import org.onosproject.store.service.EventuallyConsistentMap;
 import org.onosproject.store.service.EventuallyConsistentMapBuilder;
@@ -159,7 +160,8 @@
     private InternalPacketProcessor processor = null;
     private InternalLinkListener linkListener = null;
     private InternalDeviceListener deviceListener = null;
-    private NetworkConfigEventHandler netcfgHandler = null;
+    private AppConfigHandler appCfgHandler = null;
+    protected XConnectHandler xConnectHandler = null;
     private McastHandler mcastHandler = null;
     private HostHandler hostHandler = null;
     private InternalEventHandler eventHandler = new InternalEventHandler();
@@ -191,11 +193,6 @@
      */
     public EventuallyConsistentMap<PortNextObjectiveStoreKey, Integer>
             portNextObjStore = null;
-    /**
-     * Per cross-connect objective ID store with VLAN ID as key.
-     */
-    public EventuallyConsistentMap<XConnectNextObjectiveStoreKey, Integer>
-            xConnectNextObjStore = null;
     // Per device, per-subnet assigned-vlans store, with (device id + subnet
     // IPv4 prefix) as key
     private EventuallyConsistentMap<SubnetAssignedVidStoreKey, VlanId>
@@ -204,7 +201,8 @@
     private EventuallyConsistentMap<String, Policy> policyStore = null;
 
     private final ConfigFactory<DeviceId, SegmentRoutingDeviceConfig> deviceConfigFactory =
-            new ConfigFactory<DeviceId, SegmentRoutingDeviceConfig>(SubjectFactories.DEVICE_SUBJECT_FACTORY,
+            new ConfigFactory<DeviceId, SegmentRoutingDeviceConfig>(
+                    SubjectFactories.DEVICE_SUBJECT_FACTORY,
                     SegmentRoutingDeviceConfig.class, "segmentrouting") {
                 @Override
                 public SegmentRoutingDeviceConfig createConfig() {
@@ -212,16 +210,26 @@
                 }
             };
     private final ConfigFactory<ApplicationId, SegmentRoutingAppConfig> appConfigFactory =
-            new ConfigFactory<ApplicationId, SegmentRoutingAppConfig>(SubjectFactories.APP_SUBJECT_FACTORY,
+            new ConfigFactory<ApplicationId, SegmentRoutingAppConfig>(
+                    SubjectFactories.APP_SUBJECT_FACTORY,
                     SegmentRoutingAppConfig.class, "segmentrouting") {
                 @Override
                 public SegmentRoutingAppConfig createConfig() {
                     return new SegmentRoutingAppConfig();
                 }
             };
-
+    private final ConfigFactory<ApplicationId, XConnectConfig> xConnectConfigFactory =
+            new ConfigFactory<ApplicationId, XConnectConfig>(
+                    SubjectFactories.APP_SUBJECT_FACTORY,
+                    XConnectConfig.class, "xconnect") {
+                @Override
+                public XConnectConfig createConfig() {
+                    return new XConnectConfig();
+                }
+            };
     private ConfigFactory<ApplicationId, McastConfig> mcastConfigFactory =
-            new ConfigFactory<ApplicationId, McastConfig>(SubjectFactories.APP_SUBJECT_FACTORY,
+            new ConfigFactory<ApplicationId, McastConfig>(
+                    SubjectFactories.APP_SUBJECT_FACTORY,
                     McastConfig.class, "multicast") {
                 @Override
                 public McastConfig createConfig() {
@@ -280,15 +288,6 @@
                 .withTimestampProvider((k, v) -> new WallClockTimestamp())
                 .build();
 
-        log.debug("Creating EC map xconnectnextobjectivestore");
-        EventuallyConsistentMapBuilder<XConnectNextObjectiveStoreKey, Integer>
-                xConnectNextObjStoreBuilder = storageService.eventuallyConsistentMapBuilder();
-        xConnectNextObjStore = xConnectNextObjStoreBuilder
-                .withName("xconnectnextobjectivestore")
-                .withSerializer(createSerializer())
-                .withTimestampProvider((k, v) -> new WallClockTimestamp())
-                .build();
-
         EventuallyConsistentMapBuilder<String, Tunnel> tunnelMapBuilder =
                 storageService.eventuallyConsistentMapBuilder();
         tunnelStore = tunnelMapBuilder
@@ -321,13 +320,15 @@
         processor = new InternalPacketProcessor();
         linkListener = new InternalLinkListener();
         deviceListener = new InternalDeviceListener();
-        netcfgHandler = new NetworkConfigEventHandler(this);
+        appCfgHandler = new AppConfigHandler(this);
+        xConnectHandler = new XConnectHandler(this);
         mcastHandler = new McastHandler(this);
         hostHandler = new HostHandler(this);
 
         cfgService.addListener(cfgListener);
         cfgService.registerConfigFactory(deviceConfigFactory);
         cfgService.registerConfigFactory(appConfigFactory);
+        cfgService.registerConfigFactory(xConnectConfigFactory);
         cfgService.registerConfigFactory(mcastConfigFactory);
         hostService.addListener(hostListener);
         packetService.addProcessor(processor, PacketProcessor.director(2));
@@ -358,7 +359,7 @@
                         TunnelPolicy.class,
                         Policy.Type.class,
                         PortNextObjectiveStoreKey.class,
-                        XConnectNextObjectiveStoreKey.class
+                        XConnectStoreKey.class
                 );
     }
 
@@ -387,7 +388,6 @@
         nsNextObjStore.destroy();
         subnetNextObjStore.destroy();
         portNextObjStore.destroy();
-        xConnectNextObjStore.destroy();
         tunnelStore.destroy();
         policyStore.destroy();
         subnetVidStore.destroy();
@@ -591,25 +591,6 @@
         }
     }
 
-    /**
-     * Returns the next objective ID of type broadcast associated with the VLAN
-     * cross-connection.
-     *
-     * @param deviceId Device ID for the cross-connection
-     * @param vlanId VLAN ID for the cross-connection
-     * @return next objective ID or -1 if it was not found
-     */
-    public int getXConnectNextObjectiveId(DeviceId deviceId, VlanId vlanId) {
-        DefaultGroupHandler ghdlr = groupHandlerMap.get(deviceId);
-        if (ghdlr != null) {
-            return ghdlr.getXConnectNextObjectiveId(vlanId);
-        } else {
-            log.warn("getPortNextObjectiveId query - groupHandler for device {}"
-                    + " not found", deviceId);
-            return -1;
-        }
-    }
-
     private class InternalPacketProcessor implements PacketProcessor {
         @Override
         public void process(PacketContext context) {
@@ -836,14 +817,13 @@
 
         if (mastershipService.isLocalMaster(deviceId)) {
             hostHandler.readInitialHosts(deviceId);
+            xConnectHandler.init(deviceId);
             DefaultGroupHandler groupHandler = groupHandlerMap.get(deviceId);
             groupHandler.createGroupsFromSubnetConfig();
             routingRulePopulator.populateSubnetBroadcastRule(deviceId);
-            groupHandler.createGroupsForXConnect(deviceId);
-            routingRulePopulator.populateXConnectBroadcastRule(deviceId);
         }
 
-        netcfgHandler.initVRouters(deviceId);
+        appCfgHandler.initVRouters(deviceId);
     }
 
     private void processDeviceRemoved(Device device) {
@@ -862,11 +842,6 @@
                 .forEach(entry -> {
                     portNextObjStore.remove(entry.getKey());
                 });
-        xConnectNextObjStore.entrySet().stream()
-                .filter(entry -> entry.getKey().deviceId().equals(device.id()))
-                .forEach(entry -> {
-                    xConnectNextObjStore.remove(entry.getKey());
-                });
         subnetVidStore.entrySet().stream()
                 .filter(entry -> entry.getKey().deviceId().equals(device.id()))
                 .forEach(entry -> {
@@ -875,6 +850,7 @@
         groupHandlerMap.remove(device.id());
         defaultRoutingHandler.purgeEcmpGraph(device.id());
         mcastHandler.removeDevice(device.id());
+        xConnectHandler.removeDevice(device.id());
     }
 
     private void processPortRemoved(Device device, Port port) {
@@ -942,16 +918,31 @@
                         break;
                 }
             } else if (event.configClass().equals(SegmentRoutingAppConfig.class)) {
-                checkState(netcfgHandler != null, "NetworkConfigEventHandler is not initialized");
+                checkState(appCfgHandler != null, "NetworkConfigEventHandler is not initialized");
                 switch (event.type()) {
                     case CONFIG_ADDED:
-                        netcfgHandler.processVRouterConfigAdded(event);
+                        appCfgHandler.processAppConfigAdded(event);
                         break;
                     case CONFIG_UPDATED:
-                        netcfgHandler.processVRouterConfigUpdated(event);
+                        appCfgHandler.processAppConfigUpdated(event);
                         break;
                     case CONFIG_REMOVED:
-                        netcfgHandler.processVRouterConfigRemoved(event);
+                        appCfgHandler.processAppConfigRemoved(event);
+                        break;
+                    default:
+                        break;
+                }
+            } else if (event.configClass().equals(XConnectConfig.class)) {
+                checkState(xConnectHandler != null, "XConnectHandler is not initialized");
+                switch (event.type()) {
+                    case CONFIG_ADDED:
+                        xConnectHandler.processXConnectConfigAdded(event);
+                        break;
+                    case CONFIG_UPDATED:
+                        xConnectHandler.processXConnectConfigUpdated(event);
+                        break;
+                    case CONFIG_REMOVED:
+                        xConnectHandler.processXConnectConfigRemoved(event);
                         break;
                     default:
                         break;
diff --git a/apps/segmentrouting/src/main/java/org/onosproject/segmentrouting/XConnectHandler.java b/apps/segmentrouting/src/main/java/org/onosproject/segmentrouting/XConnectHandler.java
new file mode 100644
index 0000000..0adf7eb
--- /dev/null
+++ b/apps/segmentrouting/src/main/java/org/onosproject/segmentrouting/XConnectHandler.java
@@ -0,0 +1,498 @@
+/*
+ * 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.segmentrouting;
+
+import com.google.common.collect.ImmutableSet;
+import org.onlab.packet.MacAddress;
+import org.onlab.util.KryoNamespace;
+import org.onosproject.net.ConnectPoint;
+import org.onosproject.net.DeviceId;
+import org.onosproject.net.PortNumber;
+import org.onosproject.net.config.NetworkConfigEvent;
+import org.onosproject.net.flow.DefaultTrafficSelector;
+import org.onosproject.net.flow.DefaultTrafficTreatment;
+import org.onosproject.net.flow.TrafficSelector;
+import org.onosproject.net.flow.TrafficTreatment;
+import org.onosproject.net.flow.criteria.Criteria;
+import org.onosproject.net.flowobjective.DefaultFilteringObjective;
+import org.onosproject.net.flowobjective.DefaultForwardingObjective;
+import org.onosproject.net.flowobjective.DefaultNextObjective;
+import org.onosproject.net.flowobjective.DefaultObjectiveContext;
+import org.onosproject.net.flowobjective.FilteringObjective;
+import org.onosproject.net.flowobjective.ForwardingObjective;
+import org.onosproject.net.flowobjective.NextObjective;
+import org.onosproject.net.flowobjective.Objective;
+import org.onosproject.net.flowobjective.ObjectiveContext;
+import org.onosproject.net.flowobjective.ObjectiveError;
+import org.onosproject.segmentrouting.config.XConnectConfig;
+import org.onosproject.segmentrouting.storekey.XConnectStoreKey;
+import org.onosproject.store.serializers.KryoNamespaces;
+import org.onosproject.store.service.ConsistentMap;
+import org.onosproject.store.service.Serializer;
+import org.onosproject.store.service.StorageService;
+import org.onosproject.store.service.Versioned;
+import org.slf4j.Logger;
+import org.slf4j.LoggerFactory;
+
+import java.util.Iterator;
+import java.util.Map;
+import java.util.Set;
+import java.util.concurrent.CompletableFuture;
+import java.util.stream.Collectors;
+
+/**
+ * Handles cross connect related events.
+ */
+public class XConnectHandler {
+    private static final Logger log = LoggerFactory.getLogger(XConnectHandler.class);
+    private static final String CONFIG_NOT_FOUND = "XConnect config missing";
+    private static final String NOT_MASTER = "Not master controller";
+    private final SegmentRoutingManager srManager;
+    private final StorageService storageService;
+    private final ConsistentMap<XConnectStoreKey, NextObjective> xConnectNextObjStore;
+    private final KryoNamespace.Builder xConnectKryo;
+
+    protected XConnectHandler(SegmentRoutingManager srManager) {
+        this.srManager = srManager;
+        this.storageService = srManager.storageService;
+        xConnectKryo = new KryoNamespace.Builder()
+                .register(KryoNamespaces.API)
+                .register(XConnectStoreKey.class)
+                .register(NextObjContext.class);
+        xConnectNextObjStore = storageService
+                .<XConnectStoreKey, NextObjective>consistentMapBuilder()
+                .withName("onos-xconnect-nextobj-store")
+                .withSerializer(Serializer.using(xConnectKryo.build()))
+                .build();
+    }
+
+    /**
+     * Read initial XConnect for given device.
+     *
+     * @param deviceId ID of the device to be initialized
+     */
+    public void init(DeviceId deviceId) {
+        // Try to read XConnect config
+        XConnectConfig config =
+                srManager.cfgService.getConfig(srManager.appId, XConnectConfig.class);
+        if (config == null) {
+            log.warn("Failed to read XConnect config: {}", CONFIG_NOT_FOUND);
+            return;
+        }
+
+        config.getXconnects(deviceId).forEach(key -> {
+            populateXConnect(key, config.getPorts(key));
+        });
+    }
+
+    /**
+     * Processes Segment Routing App Config added event.
+     *
+     * @param event network config added event
+     */
+    protected void processXConnectConfigAdded(NetworkConfigEvent event) {
+        log.info("Processing XConnect CONFIG_ADDED");
+        XConnectConfig config = (XConnectConfig) event.config().get();
+        config.getXconnects().forEach(key -> {
+            populateXConnect(key, config.getPorts(key));
+        });
+    }
+
+    /**
+     * Processes Segment Routing App Config updated event.
+     *
+     * @param event network config updated event
+     */
+    protected void processXConnectConfigUpdated(NetworkConfigEvent event) {
+        log.info("Processing XConnect CONFIG_UPDATED");
+        XConnectConfig prevConfig = (XConnectConfig) event.prevConfig().get();
+        XConnectConfig config = (XConnectConfig) event.config().get();
+        Set<XConnectStoreKey> prevKeys = prevConfig.getXconnects();
+        Set<XConnectStoreKey> keys = config.getXconnects();
+
+        Set<XConnectStoreKey> pendingRemove = prevKeys.stream()
+                .filter(key -> !keys.contains(key)).collect(Collectors.toSet());
+        Set<XConnectStoreKey> pendingAdd = keys.stream()
+                .filter(key -> !prevKeys.contains(key)).collect(Collectors.toSet());
+        Set<XConnectStoreKey> pendingUpdate = keys.stream()
+                .filter(key -> prevKeys.contains(key) &&
+                        !config.getPorts(key).equals(prevConfig.getPorts(key)))
+                .collect(Collectors.toSet());
+
+        pendingRemove.forEach(key -> {
+            revokeXConnect(key, prevConfig.getPorts(key));
+        });
+        pendingAdd.forEach(key -> {
+            populateXConnect(key, config.getPorts(key));
+        });
+        pendingUpdate.forEach(key -> {
+            updateXConnect(key, prevConfig.getPorts(key), config.getPorts(key));
+        });
+    }
+
+    /**
+     * Processes Segment Routing App Config removed event.
+     *
+     * @param event network config removed event
+     */
+    protected void processXConnectConfigRemoved(NetworkConfigEvent event) {
+        log.info("Processing XConnect CONFIG_REMOVED");
+        XConnectConfig prevConfig = (XConnectConfig) event.prevConfig().get();
+        prevConfig.getXconnects().forEach(key -> {
+            revokeXConnect(key, prevConfig.getPorts(key));
+        });
+    }
+
+    /**
+     * Checks if there is any XConnect configured on given connect point.
+     *
+     * @param cp connect point
+     * @return true if there is XConnect configured on given connect point.
+     */
+    public boolean hasXConnect(ConnectPoint cp) {
+        // Try to read XConnect config
+        XConnectConfig config =
+                srManager.cfgService.getConfig(srManager.appId, XConnectConfig.class);
+        if (config == null) {
+            log.warn("Failed to read XConnect config: {}", CONFIG_NOT_FOUND);
+            return false;
+        }
+        return config.getXconnects(cp.deviceId()).stream()
+                .anyMatch(key -> config.getPorts(key).contains(cp.port()));
+    }
+
+    /**
+     * Populates XConnect groups and flows for given key.
+     *
+     * @param key XConnect key
+     * @param ports a set of ports to be cross-connected
+     */
+    private void populateXConnect(XConnectStoreKey key, Set<PortNumber> ports) {
+        if (!srManager.mastershipService.isLocalMaster(key.deviceId())) {
+            log.info("Abort populating XConnect {}: {}", key, NOT_MASTER);
+            return;
+        }
+        populateFilter(key, ports);
+        populateFwd(key, populateNext(key, ports));
+    }
+
+    /**
+     * Populates filtering objectives for given XConnect.
+     *
+     * @param key XConnect store key
+     * @param ports XConnect ports
+     */
+    private void populateFilter(XConnectStoreKey key, Set<PortNumber> ports) {
+        ports.forEach(port -> {
+            FilteringObjective.Builder filtObjBuilder = filterObjBuilder(key, port);
+            ObjectiveContext context = new DefaultObjectiveContext(
+                    (objective) -> log.debug("XConnect FilterObj for {} on port {} populated",
+                            key, port),
+                    (objective, error) ->
+                            log.warn("Failed to populate XConnect FilterObj for {} on port {}: {}",
+                                    key, port, error));
+            srManager.flowObjectiveService.filter(key.deviceId(), filtObjBuilder.add(context));
+        });
+    }
+
+    /**
+     * Populates next objectives for given XConnect.
+     *
+     * @param key XConnect store key
+     * @param ports XConnect ports
+     */
+    private NextObjective populateNext(XConnectStoreKey key, Set<PortNumber> ports) {
+        NextObjective nextObj = null;
+        if (xConnectNextObjStore.containsKey(key)) {
+            nextObj = xConnectNextObjStore.get(key).value();
+            log.debug("NextObj for {} found, id={}", key, nextObj.id());
+        } else {
+            NextObjective.Builder nextObjBuilder = nextObjBuilder(key, ports);
+            ObjectiveContext nextContext = new NextObjContext(Objective.Operation.ADD, key);
+            nextObj = nextObjBuilder.add(nextContext);
+            srManager.flowObjectiveService.next(key.deviceId(), nextObj);
+            xConnectNextObjStore.put(key, nextObj);
+            log.debug("NextObj for {} not found. Creating new NextObj with id={}", key, nextObj.id());
+        }
+        return nextObj;
+    }
+
+    /**
+     * Populates forwarding objectives for given XConnect.
+     *
+     * @param key XConnect store key
+     * @param nextObj next objective
+     */
+    private void populateFwd(XConnectStoreKey key, NextObjective nextObj) {
+        ForwardingObjective.Builder fwdObjBuilder = fwdObjBuilder(key, nextObj.id());
+        ObjectiveContext fwdContext = new DefaultObjectiveContext(
+                (objective) -> log.debug("XConnect FwdObj for {} populated", key),
+                (objective, error) ->
+                        log.warn("Failed to populate XConnect FwdObj for {}: {}", key, error));
+        srManager.flowObjectiveService.forward(key.deviceId(), fwdObjBuilder.add(fwdContext));
+    }
+
+    /**
+     * Revokes XConnect groups and flows for given key.
+     *
+     * @param key XConnect key
+     * @param ports XConnect ports
+     */
+    private void revokeXConnect(XConnectStoreKey key, Set<PortNumber> ports) {
+        if (!srManager.mastershipService.isLocalMaster(key.deviceId())) {
+            log.info("Abort populating XConnect {}: {}", key, NOT_MASTER);
+            return;
+        }
+
+        revokeFilter(key, ports);
+        if (xConnectNextObjStore.containsKey(key)) {
+            NextObjective nextObj = xConnectNextObjStore.get(key).value();
+            revokeFwd(key, nextObj, null);
+            revokeNext(key, nextObj, null);
+        } else {
+            log.warn("NextObj for {} does not exist in the store.", key);
+        }
+    }
+
+    /**
+     * Revokes filtering objectives for given XConnect.
+     *
+     * @param key XConnect store key
+     * @param ports XConnect ports
+     */
+    private void revokeFilter(XConnectStoreKey key, Set<PortNumber> ports) {
+        ports.forEach(port -> {
+            FilteringObjective.Builder filtObjBuilder = filterObjBuilder(key, port);
+            ObjectiveContext context = new DefaultObjectiveContext(
+                    (objective) -> log.debug("XConnect FilterObj for {} on port {} revoked",
+                            key, port),
+                    (objective, error) ->
+                            log.warn("Failed to revoke XConnect FilterObj for {} on port {}: {}",
+                                    key, port, error));
+            srManager.flowObjectiveService.filter(key.deviceId(), filtObjBuilder.remove(context));
+        });
+    }
+
+    /**
+     * Revokes next objectives for given XConnect.
+     *
+     * @param key XConnect store key
+     * @param nextObj next objective
+     * @param nextFuture completable future for this next objective operation
+     */
+    private void revokeNext(XConnectStoreKey key, NextObjective nextObj,
+            CompletableFuture<ObjectiveError> nextFuture) {
+        ObjectiveContext context = new ObjectiveContext() {
+            @Override
+            public void onSuccess(Objective objective) {
+                log.debug("Previous NextObj for {} removed", key);
+                if (nextFuture != null) {
+                    nextFuture.complete(null);
+                }
+            }
+
+            @Override
+            public void onError(Objective objective, ObjectiveError error) {
+                log.warn("Failed to remove previous NextObj for {}: {}", key, error);
+                if (nextFuture != null) {
+                    nextFuture.complete(error);
+                }
+            }
+        };
+        srManager.flowObjectiveService.next(key.deviceId(),
+                (NextObjective) nextObj.copy().remove(context));
+        xConnectNextObjStore.remove(key);
+    }
+
+    /**
+     * Revokes forwarding objectives for given XConnect.
+     *
+     * @param key XConnect store key
+     * @param nextObj next objective
+     * @param fwdFuture completable future for this forwarding objective operation
+     */
+    private void revokeFwd(XConnectStoreKey key, NextObjective nextObj,
+            CompletableFuture<ObjectiveError> fwdFuture) {
+        ForwardingObjective.Builder fwdObjBuilder = fwdObjBuilder(key, nextObj.id());
+        ObjectiveContext context = new ObjectiveContext() {
+            @Override
+            public void onSuccess(Objective objective) {
+                log.debug("Previous FwdObj for {} removed", key);
+                if (fwdFuture != null) {
+                    fwdFuture.complete(null);
+                }
+            }
+
+            @Override
+            public void onError(Objective objective, ObjectiveError error) {
+                log.warn("Failed to remove previous FwdObj for {}: {}", key, error);
+                if (fwdFuture != null) {
+                    fwdFuture.complete(error);
+                }
+            }
+        };
+        srManager.flowObjectiveService
+                .forward(key.deviceId(), fwdObjBuilder.remove(context));
+    }
+
+    /**
+     * Updates XConnect groups and flows for given key.
+     *
+     * @param key XConnect key
+     * @param prevPorts previous XConnect ports
+     * @param ports new XConnect ports
+     */
+    private void updateXConnect(XConnectStoreKey key, Set<PortNumber> prevPorts,
+            Set<PortNumber> ports) {
+        // remove old filter
+        prevPorts.stream().filter(port -> !ports.contains(port)).forEach(port -> {
+            revokeFilter(key, ImmutableSet.of(port));
+        });
+        // install new filter
+        ports.stream().filter(port -> !prevPorts.contains(port)).forEach(port -> {
+            populateFilter(key, ImmutableSet.of(port));
+        });
+
+        CompletableFuture<ObjectiveError> fwdFuture = new CompletableFuture<>();
+        CompletableFuture<ObjectiveError> nextFuture = new CompletableFuture<>();
+
+        if (xConnectNextObjStore.containsKey(key)) {
+            NextObjective nextObj = xConnectNextObjStore.get(key).value();
+            revokeFwd(key, nextObj, fwdFuture);
+
+            fwdFuture.thenAcceptAsync(fwdStatus -> {
+                if (fwdStatus == null) {
+                    log.debug("Fwd removed. Now remove group {}", key);
+                    revokeNext(key, nextObj, nextFuture);
+                }
+            });
+
+            nextFuture.thenAcceptAsync(nextStatus -> {
+                if (nextStatus == null) {
+                    log.debug("Installing new group and flow for {}", key);
+                    populateFwd(key, populateNext(key, ports));
+                }
+            });
+        } else {
+            log.warn("NextObj for {} does not exist in the store.", key);
+        }
+    }
+
+    /**
+     * Remove all groups on given device.
+     *
+     * @param deviceId device ID
+     */
+    protected void removeDevice(DeviceId deviceId) {
+        Iterator<Map.Entry<XConnectStoreKey, Versioned<NextObjective>>> itNextObj =
+                xConnectNextObjStore.entrySet().iterator();
+        while (itNextObj.hasNext()) {
+            Map.Entry<XConnectStoreKey, Versioned<NextObjective>> entry = itNextObj.next();
+            if (entry.getKey().deviceId().equals(deviceId)) {
+                itNextObj.remove();
+            }
+        }
+    }
+
+    /**
+     * Creates a next objective builder for XConnect.
+     *
+     * @param key XConnect key
+     * @param ports set of XConnect ports
+     * @return next objective builder
+     */
+    private NextObjective.Builder nextObjBuilder(XConnectStoreKey key, Set<PortNumber> ports) {
+        int nextId = srManager.flowObjectiveService.allocateNextId();
+        TrafficSelector metadata =
+                DefaultTrafficSelector.builder().matchVlanId(key.vlanId()).build();
+        NextObjective.Builder nextObjBuilder = DefaultNextObjective
+                .builder().withId(nextId)
+                .withType(NextObjective.Type.BROADCAST).fromApp(srManager.appId)
+                .withMeta(metadata);
+        ports.forEach(port -> {
+            TrafficTreatment.Builder tBuilder = DefaultTrafficTreatment.builder();
+            tBuilder.setOutput(port);
+            nextObjBuilder.addTreatment(tBuilder.build());
+        });
+        return nextObjBuilder;
+    }
+
+    /**
+     * Creates a forwarding objective builder for XConnect.
+     *
+     * @param key XConnect key
+     * @param nextId next ID of the broadcast group for this XConnect key
+     * @return next objective builder
+     */
+    private ForwardingObjective.Builder fwdObjBuilder(XConnectStoreKey key, int nextId) {
+        /*
+         * Driver should treat objectives with MacAddress.NONE and !VlanId.NONE
+         * as the VLAN cross-connect broadcast rules
+         */
+        TrafficSelector.Builder sbuilder = DefaultTrafficSelector.builder();
+        sbuilder.matchVlanId(key.vlanId());
+        sbuilder.matchEthDst(MacAddress.NONE);
+
+        ForwardingObjective.Builder fob = DefaultForwardingObjective.builder();
+        fob.withFlag(ForwardingObjective.Flag.SPECIFIC)
+                .withSelector(sbuilder.build())
+                .nextStep(nextId)
+                .withPriority(SegmentRoutingService.XCONNECT_PRIORITY)
+                .fromApp(srManager.appId)
+                .makePermanent();
+        return fob;
+    }
+
+    /**
+     * Creates a filtering objective builder for XConnect.
+     *
+     * @param key XConnect key
+     * @param port XConnect ports
+     * @return next objective builder
+     */
+    private FilteringObjective.Builder filterObjBuilder(XConnectStoreKey key, PortNumber port) {
+        FilteringObjective.Builder fob = DefaultFilteringObjective.builder();
+        fob.withKey(Criteria.matchInPort(port))
+                .addCondition(Criteria.matchVlanId(key.vlanId()))
+                .addCondition(Criteria.matchEthDst(MacAddress.NONE))
+                .withPriority(SegmentRoutingService.XCONNECT_PRIORITY);
+        return fob.permit().fromApp(srManager.appId);
+    }
+
+    // TODO: Lambda closure in DefaultObjectiveContext cannot be serialized properly
+    //       with Kryo 3.0.3. It will be fixed in 3.0.4. By then we can use
+    //       DefaultObjectiveContext again.
+    private final class NextObjContext implements ObjectiveContext {
+        Objective.Operation op;
+        XConnectStoreKey key;
+
+        private NextObjContext(Objective.Operation op, XConnectStoreKey key) {
+            this.op = op;
+            this.key = key;
+        }
+
+        @Override
+        public void onSuccess(Objective objective) {
+            log.debug("XConnect NextObj for {} {}ED", key, op);
+        }
+
+        @Override
+        public void onError(Objective objective, ObjectiveError error) {
+            log.warn("Failed to {} XConnect NextObj for {}: {}", op, key, error);
+        }
+    }
+}
diff --git a/apps/segmentrouting/src/main/java/org/onosproject/segmentrouting/config/DeviceConfiguration.java b/apps/segmentrouting/src/main/java/org/onosproject/segmentrouting/config/DeviceConfiguration.java
index 4474fb2..c1af6f2 100644
--- a/apps/segmentrouting/src/main/java/org/onosproject/segmentrouting/config/DeviceConfiguration.java
+++ b/apps/segmentrouting/src/main/java/org/onosproject/segmentrouting/config/DeviceConfiguration.java
@@ -39,7 +39,6 @@
 import java.util.ArrayList;
 import java.util.HashMap;
 import java.util.HashSet;
-import java.util.LinkedList;
 import java.util.List;
 import java.util.Map;
 import java.util.Optional;
@@ -58,7 +57,6 @@
     private static final Logger log = LoggerFactory.getLogger(DeviceConfiguration.class);
     private final List<Integer> allSegmentIds = new ArrayList<>();
     private final Map<DeviceId, SegmentRouterInfo> deviceConfigMap = new ConcurrentHashMap<>();
-    private final Map<VlanId, List<ConnectPoint>> xConnects = new ConcurrentHashMap<>();
     private ApplicationId appId;
     private NetworkConfigService cfgService;
 
@@ -148,28 +146,6 @@
                         }
                         info.subnets.put(port, interfaceAddress.subnetAddress().getIp4Prefix());
                     });
-
-                    // Extract VLAN cross-connect information
-                    // Do not setup cross-connect if VLAN is NONE
-                    if (vlanId.equals(VlanId.NONE)) {
-                        return;
-                    }
-                    List<ConnectPoint> connectPoints = xConnects.get(vlanId);
-                    if (connectPoints != null) {
-                        if (connectPoints.size() != 1) {
-                            log.warn("Cross-connect should only have two endpoints. Aborting.");
-                            return;
-                        }
-                        if (!connectPoints.get(0).deviceId().equals(connectPoint.deviceId())) {
-                            log.warn("Cross-connect endpoints must be on the same switch. Aborting.");
-                            return;
-                        }
-                        connectPoints.add(connectPoint);
-                    } else {
-                        connectPoints = new LinkedList<>();
-                        connectPoints.add(connectPoint);
-                        xConnects.put(vlanId, connectPoints);
-                    }
                 }
             });
         });
@@ -298,11 +274,6 @@
         return subnetPortMap;
     }
 
-    @Override
-    public Map<VlanId, List<ConnectPoint>> getXConnects() {
-        return xConnects;
-    }
-
     /**
      * Returns the device identifier or data plane identifier (dpid)
      * of a segment router given its segment id.
diff --git a/apps/segmentrouting/src/main/java/org/onosproject/segmentrouting/config/DeviceProperties.java b/apps/segmentrouting/src/main/java/org/onosproject/segmentrouting/config/DeviceProperties.java
index 967d2b6..5ad9cdc 100644
--- a/apps/segmentrouting/src/main/java/org/onosproject/segmentrouting/config/DeviceProperties.java
+++ b/apps/segmentrouting/src/main/java/org/onosproject/segmentrouting/config/DeviceProperties.java
@@ -21,8 +21,6 @@
 import org.onlab.packet.Ip4Address;
 import org.onlab.packet.Ip4Prefix;
 import org.onlab.packet.MacAddress;
-import org.onlab.packet.VlanId;
-import org.onosproject.net.ConnectPoint;
 import org.onosproject.net.DeviceId;
 import org.onosproject.net.PortNumber;
 
@@ -97,11 +95,4 @@
      */
     Map<Ip4Prefix, List<PortNumber>> getSubnetPortsMap(DeviceId deviceId)
             throws DeviceConfigNotFoundException;
-
-    /**
-     * Returns the VLAN cross-connect configuration.
-     *
-     * @return A map of that maps VLAN ID to a list of cross-connect endpoints
-     */
-    Map<VlanId, List<ConnectPoint>> getXConnects();
 }
diff --git a/apps/segmentrouting/src/main/java/org/onosproject/segmentrouting/config/XConnectConfig.java b/apps/segmentrouting/src/main/java/org/onosproject/segmentrouting/config/XConnectConfig.java
new file mode 100644
index 0000000..089f8b0
--- /dev/null
+++ b/apps/segmentrouting/src/main/java/org/onosproject/segmentrouting/config/XConnectConfig.java
@@ -0,0 +1,111 @@
+/*
+ * 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.segmentrouting.config;
+
+import com.fasterxml.jackson.databind.JsonNode;
+import com.fasterxml.jackson.databind.node.ObjectNode;
+import com.google.common.collect.ImmutableSet;
+import org.onlab.packet.VlanId;
+import org.onosproject.core.ApplicationId;
+import org.onosproject.net.DeviceId;
+import org.onosproject.net.PortNumber;
+import org.onosproject.net.config.Config;
+import org.onosproject.segmentrouting.storekey.XConnectStoreKey;
+
+import java.util.Set;
+
+import static com.google.common.base.Preconditions.checkArgument;
+
+/**
+ * Configuration object for cross-connect.
+ */
+public class XConnectConfig extends Config<ApplicationId> {
+    private static final String VLAN = "vlan";
+    private static final String PORTS = "ports";
+    private static final String NAME = "name"; // dummy field for naming
+
+    private static final String UNEXPECTED_FIELD_NAME = "Unexpected field name";
+
+    @Override
+    public boolean isValid() {
+        try {
+            getXconnects().forEach(this::getPorts);
+        } catch (IllegalArgumentException e) {
+            return false;
+        }
+        return true;
+    }
+
+    /**
+     * Returns all xconnect keys.
+     *
+     * @return all keys (device/vlan pairs)
+     * @throws IllegalArgumentException if wrong format
+     */
+    public Set<XConnectStoreKey> getXconnects() {
+        ImmutableSet.Builder<XConnectStoreKey> builder = ImmutableSet.builder();
+        object.fields().forEachRemaining(entry -> {
+            DeviceId deviceId = DeviceId.deviceId(entry.getKey());
+            builder.addAll(getXconnects(deviceId));
+        });
+        return builder.build();
+    }
+
+    /**
+     * Returns xconnect keys of given device.
+     *
+     * @param deviceId ID of the device from which we want to get XConnect info
+     * @return xconnect keys (device/vlan pairs) of given device
+     * @throws IllegalArgumentException if wrong format
+     */
+    public Set<XConnectStoreKey> getXconnects(DeviceId deviceId) {
+        ImmutableSet.Builder<XConnectStoreKey> builder = ImmutableSet.builder();
+        JsonNode vlanPortPair = object.get(deviceId.toString());
+        if (vlanPortPair != null) {
+            vlanPortPair.forEach(jsonNode -> {
+                if (!hasOnlyFields((ObjectNode) jsonNode, VLAN, PORTS, NAME)) {
+                    throw new IllegalArgumentException(UNEXPECTED_FIELD_NAME);
+                }
+                VlanId vlanId = VlanId.vlanId((short) jsonNode.get(VLAN).asInt());
+                builder.add(new XConnectStoreKey(deviceId, vlanId));
+            });
+        }
+        return builder.build();
+    }
+
+    /**
+     * Returns ports of given xconnect key.
+     *
+     * @param xconnect xconnect key
+     * @return set of two ports associated with given xconnect key
+     * @throws IllegalArgumentException if wrong format
+     */
+    public Set<PortNumber> getPorts(XConnectStoreKey xconnect) {
+        ImmutableSet.Builder<PortNumber> builder = ImmutableSet.builder();
+        object.get(xconnect.deviceId().toString()).forEach(vlanPortsPair -> {
+            if (xconnect.vlanId().toShort() == vlanPortsPair.get(VLAN).asInt()) {
+                int portCount = vlanPortsPair.get(PORTS).size();
+                checkArgument(portCount == 2,
+                        "Expect 2 ports but found " + portCount + " on " + xconnect);
+                vlanPortsPair.get(PORTS).forEach(portNode -> {
+                    builder.add(PortNumber.portNumber(portNode.asInt()));
+                });
+            }
+        });
+        return builder.build();
+    }
+}
diff --git a/apps/segmentrouting/src/main/java/org/onosproject/segmentrouting/grouphandler/DefaultGroupHandler.java b/apps/segmentrouting/src/main/java/org/onosproject/segmentrouting/grouphandler/DefaultGroupHandler.java
index 1399153..e82049b 100644
--- a/apps/segmentrouting/src/main/java/org/onosproject/segmentrouting/grouphandler/DefaultGroupHandler.java
+++ b/apps/segmentrouting/src/main/java/org/onosproject/segmentrouting/grouphandler/DefaultGroupHandler.java
@@ -35,7 +35,6 @@
 import org.onlab.packet.VlanId;
 import org.onlab.util.KryoNamespace;
 import org.onosproject.core.ApplicationId;
-import org.onosproject.net.ConnectPoint;
 import org.onosproject.net.DeviceId;
 import org.onosproject.net.Link;
 import org.onosproject.net.PortNumber;
@@ -55,7 +54,6 @@
 import org.onosproject.segmentrouting.storekey.NeighborSetNextObjectiveStoreKey;
 import org.onosproject.segmentrouting.storekey.PortNextObjectiveStoreKey;
 import org.onosproject.segmentrouting.storekey.SubnetNextObjectiveStoreKey;
-import org.onosproject.segmentrouting.storekey.XConnectNextObjectiveStoreKey;
 import org.onosproject.store.service.EventuallyConsistentMap;
 import org.slf4j.Logger;
 
@@ -89,8 +87,6 @@
             subnetNextObjStore = null;
     protected EventuallyConsistentMap<PortNextObjectiveStoreKey, Integer>
             portNextObjStore = null;
-    protected EventuallyConsistentMap<XConnectNextObjectiveStoreKey, Integer>
-            xConnectNextObjStore = null;
     private SegmentRoutingManager srManager;
 
     protected KryoNamespace.Builder kryo = new KryoNamespace.Builder()
@@ -123,7 +119,6 @@
         this.nsNextObjStore = srManager.nsNextObjStore;
         this.subnetNextObjStore = srManager.subnetNextObjStore;
         this.portNextObjStore = srManager.portNextObjStore;
-        this.xConnectNextObjStore = srManager.xConnectNextObjStore;
         this.srManager = srManager;
 
         populateNeighborMaps();
@@ -471,32 +466,6 @@
     }
 
     /**
-     * Returns the next objective ID of type broadcast associated with the VLAN
-     * cross-connection.
-     *
-     * @param vlanId VLAN ID for the cross-connection
-     * @return int if found or created, -1 if there are errors during the
-     *         creation of the next objective
-     */
-    public int getXConnectNextObjectiveId(VlanId vlanId) {
-        Integer nextId = xConnectNextObjStore
-                .get(new XConnectNextObjectiveStoreKey(deviceId, vlanId));
-        if (nextId == null) {
-            log.trace("getXConnectNextObjectiveId: Next objective id "
-                    + "not found for device {} and vlan {}. Creating", deviceId, vlanId);
-            createGroupsForXConnect(deviceId);
-            nextId = xConnectNextObjStore.get(
-                    new XConnectNextObjectiveStoreKey(deviceId, vlanId));
-            if (nextId == null) {
-                log.warn("getXConnectNextObjectiveId: Next objective id "
-                        + "not found for device {} and vlan {}.", deviceId, vlanId);
-                return -1;
-            }
-        }
-        return nextId;
-    }
-
-    /**
      * Checks if the next objective ID (group) for the neighbor set exists or not.
      *
      * @param ns neighbor set to check
@@ -743,55 +712,6 @@
     }
 
     /**
-     * Creates broadcast groups for VLAN cross-connect ports.
-     *
-     * @param deviceId the DPID of the switch
-     */
-    public void createGroupsForXConnect(DeviceId deviceId) {
-        Map<VlanId, List<ConnectPoint>> xConnectsForDevice = deviceConfig.getXConnects();
-
-        xConnectsForDevice.forEach((vlanId, connectPoints) -> {
-            // Only proceed  the xConnect for given device
-            for (ConnectPoint connectPoint : connectPoints) {
-                if (!connectPoint.deviceId().equals(deviceId)) {
-                    return;
-                }
-            }
-
-            // Check if the next obj is already in the store
-            XConnectNextObjectiveStoreKey key =
-                    new XConnectNextObjectiveStoreKey(deviceId, vlanId);
-            if (xConnectNextObjStore.containsKey(key)) {
-                log.debug("Cross-connect Broadcast group for device {} and vlanId {} exists",
-                        deviceId, vlanId);
-                return;
-            }
-
-            TrafficSelector metadata =
-                    DefaultTrafficSelector.builder().matchVlanId(vlanId).build();
-            int nextId = flowObjectiveService.allocateNextId();
-
-            NextObjective.Builder nextObjBuilder = DefaultNextObjective
-                    .builder().withId(nextId)
-                    .withType(NextObjective.Type.BROADCAST).fromApp(appId)
-                    .withMeta(metadata);
-
-            connectPoints.forEach(connectPoint -> {
-                TrafficTreatment.Builder tBuilder = DefaultTrafficTreatment.builder();
-                tBuilder.setOutput(connectPoint.port());
-                nextObjBuilder.addTreatment(tBuilder.build());
-            });
-
-            NextObjective nextObj = nextObjBuilder.add();
-            flowObjectiveService.next(deviceId, nextObj);
-            log.debug("createGroupsForXConnect: Submited next objective {} in device {}",
-                    nextId, deviceId);
-            xConnectNextObjStore.put(key, nextId);
-        });
-    }
-
-
-    /**
      * Create simple next objective for a single port. The treatments can include
      * all outgoing actions that need to happen on the packet.
      *
diff --git a/apps/segmentrouting/src/main/java/org/onosproject/segmentrouting/storekey/XConnectNextObjectiveStoreKey.java b/apps/segmentrouting/src/main/java/org/onosproject/segmentrouting/storekey/XConnectStoreKey.java
similarity index 87%
rename from apps/segmentrouting/src/main/java/org/onosproject/segmentrouting/storekey/XConnectNextObjectiveStoreKey.java
rename to apps/segmentrouting/src/main/java/org/onosproject/segmentrouting/storekey/XConnectStoreKey.java
index e0232bc..3849624 100644
--- a/apps/segmentrouting/src/main/java/org/onosproject/segmentrouting/storekey/XConnectNextObjectiveStoreKey.java
+++ b/apps/segmentrouting/src/main/java/org/onosproject/segmentrouting/storekey/XConnectStoreKey.java
@@ -24,7 +24,7 @@
 /**
  * Key of VLAN cross-connect next objective store.
  */
-public class XConnectNextObjectiveStoreKey {
+public class XConnectStoreKey {
     private final DeviceId deviceId;
     private final VlanId vlanId;
 
@@ -34,7 +34,7 @@
      * @param deviceId device ID of the VLAN cross-connection
      * @param vlanId VLAN ID of the VLAN cross-connection
      */
-    public XConnectNextObjectiveStoreKey(DeviceId deviceId, VlanId vlanId) {
+    public XConnectStoreKey(DeviceId deviceId, VlanId vlanId) {
         this.deviceId = deviceId;
         this.vlanId = vlanId;
     }
@@ -62,11 +62,11 @@
         if (this == o) {
             return true;
         }
-        if (!(o instanceof XConnectNextObjectiveStoreKey)) {
+        if (!(o instanceof XConnectStoreKey)) {
             return false;
         }
-        XConnectNextObjectiveStoreKey that =
-                (XConnectNextObjectiveStoreKey) o;
+        XConnectStoreKey that =
+                (XConnectStoreKey) o;
         return (Objects.equals(this.deviceId, that.deviceId) &&
                 Objects.equals(this.vlanId, that.vlanId));
     }
diff --git a/apps/segmentrouting/src/test/java/org/onosproject/segmentrouting/config/SegmentRoutingAppConfigTest.java b/apps/segmentrouting/src/test/java/org/onosproject/segmentrouting/config/SegmentRoutingAppConfigTest.java
index e6f4912..a5a7268 100644
--- a/apps/segmentrouting/src/test/java/org/onosproject/segmentrouting/config/SegmentRoutingAppConfigTest.java
+++ b/apps/segmentrouting/src/test/java/org/onosproject/segmentrouting/config/SegmentRoutingAppConfigTest.java
@@ -41,9 +41,6 @@
  * Tests for class {@link SegmentRoutingAppConfig}.
  */
 public class SegmentRoutingAppConfigTest {
-    private static final ApplicationId APP_ID =
-            new TestApplicationId(SegmentRoutingManager.SR_APP_ID);
-
     private SegmentRoutingAppConfig config;
     private SegmentRoutingAppConfig invalidConfig;
 
@@ -67,12 +64,12 @@
     @Before
     public void setUp() throws Exception {
         InputStream jsonStream = SegmentRoutingAppConfigTest.class
-                .getResourceAsStream("/sr-app-config.json");
+                .getResourceAsStream("/app.json");
         InputStream invalidJsonStream = SegmentRoutingAppConfigTest.class
-                .getResourceAsStream("/sr-app-config-invalid.json");
+                .getResourceAsStream("/app-invalid.json");
 
-        ApplicationId subject = APP_ID;
         String key = SegmentRoutingManager.SR_APP_ID;
+        ApplicationId subject = new TestApplicationId(key);
         ObjectMapper mapper = new ObjectMapper();
         JsonNode jsonNode = mapper.readTree(jsonStream);
         JsonNode invalidJsonNode = mapper.readTree(invalidJsonStream);
diff --git a/apps/segmentrouting/src/test/java/org/onosproject/segmentrouting/config/SegmentRoutingDeviceConfigTest.java b/apps/segmentrouting/src/test/java/org/onosproject/segmentrouting/config/SegmentRoutingDeviceConfigTest.java
index 7c12a2b..74f6498 100644
--- a/apps/segmentrouting/src/test/java/org/onosproject/segmentrouting/config/SegmentRoutingDeviceConfigTest.java
+++ b/apps/segmentrouting/src/test/java/org/onosproject/segmentrouting/config/SegmentRoutingDeviceConfigTest.java
@@ -47,7 +47,7 @@
     @Before
     public void setUp() throws Exception {
         InputStream jsonStream = SegmentRoutingDeviceConfigTest.class
-                .getResourceAsStream("/sr-device-config.json");
+                .getResourceAsStream("/device.json");
 
         adjacencySids1 = new HashMap<>();
         Set<Integer> ports1 = new HashSet<>();
diff --git a/apps/segmentrouting/src/test/java/org/onosproject/segmentrouting/config/XConnectConfigTest.java b/apps/segmentrouting/src/test/java/org/onosproject/segmentrouting/config/XConnectConfigTest.java
new file mode 100644
index 0000000..f472a90
--- /dev/null
+++ b/apps/segmentrouting/src/test/java/org/onosproject/segmentrouting/config/XConnectConfigTest.java
@@ -0,0 +1,128 @@
+/*
+ * 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.segmentrouting.config;
+
+import com.fasterxml.jackson.databind.JsonNode;
+import com.fasterxml.jackson.databind.ObjectMapper;
+import org.junit.Before;
+import org.junit.Test;
+import org.onlab.packet.VlanId;
+import org.onosproject.TestApplicationId;
+import org.onosproject.core.ApplicationId;
+import org.onosproject.net.DeviceId;
+import org.onosproject.net.PortNumber;
+import org.onosproject.net.config.Config;
+import org.onosproject.net.config.ConfigApplyDelegate;
+import org.onosproject.segmentrouting.SegmentRoutingManager;
+import org.onosproject.segmentrouting.storekey.XConnectStoreKey;
+import java.io.InputStream;
+import java.util.Set;
+import static org.junit.Assert.assertFalse;
+import static org.junit.Assert.assertTrue;
+import static org.junit.Assert.assertThat;
+import static org.hamcrest.Matchers.is;
+
+/**
+ * Tests for class {@link XConnectConfig}.
+ */
+public class XConnectConfigTest {
+    private static final DeviceId DEV1 = DeviceId.deviceId("of:0000000000000001");
+    private static final DeviceId DEV2 = DeviceId.deviceId("of:0000000000000002");
+    private static final VlanId VLAN10 = VlanId.vlanId((short) 10);
+    private static final VlanId VLAN20 = VlanId.vlanId((short) 20);
+    private static final PortNumber PORT3 = PortNumber.portNumber(3);
+    private static final PortNumber PORT4 = PortNumber.portNumber(4);
+    private static final PortNumber PORT5 = PortNumber.portNumber(5);
+    private static final XConnectStoreKey KEY1 = new XConnectStoreKey(DEV1, VLAN10);
+    private static final XConnectStoreKey KEY2 = new XConnectStoreKey(DEV2, VLAN10);
+    private static final XConnectStoreKey KEY3 = new XConnectStoreKey(DEV2, VLAN20);
+    private static final XConnectStoreKey KEY4 = new XConnectStoreKey(DEV2, VlanId.NONE);
+
+    private XConnectConfig config;
+    private XConnectConfig invalidConfig;
+
+    @Before
+    public void setUp() throws Exception {
+        InputStream jsonStream = SegmentRoutingAppConfigTest.class
+                .getResourceAsStream("/xconnect.json");
+        InputStream invalidJsonStream = SegmentRoutingAppConfigTest.class
+                .getResourceAsStream("/xconnect-invalid.json");
+
+        String key = SegmentRoutingManager.SR_APP_ID;
+        ApplicationId subject = new TestApplicationId(key);
+        ObjectMapper mapper = new ObjectMapper();
+        JsonNode jsonNode = mapper.readTree(jsonStream);
+        JsonNode invalidJsonNode = mapper.readTree(invalidJsonStream);
+        ConfigApplyDelegate delegate = new XConnectConfigTest.MockDelegate();
+
+        config = new XConnectConfig();
+        config.init(subject, key, jsonNode, mapper, delegate);
+        invalidConfig = new XConnectConfig();
+        invalidConfig.init(subject, key, invalidJsonNode, mapper, delegate);
+    }
+
+    /**
+     * Tests config validity.
+     */
+    @Test
+    public void testIsValid() {
+        assertTrue(config.isValid());
+        assertFalse(invalidConfig.isValid());
+    }
+
+    /**
+     * Tests getXconnects.
+     */
+    @Test
+    public void testGetXconnects() {
+        Set<XConnectStoreKey> xconnects = config.getXconnects();
+        assertThat(xconnects.size(), is(3));
+        assertTrue(xconnects.contains(KEY1));
+        assertTrue(xconnects.contains(KEY2));
+        assertTrue(xconnects.contains(KEY3));
+        assertFalse(xconnects.contains(KEY4));
+    }
+
+    /**
+     * Tests getPorts.
+     */
+    @Test
+    public void testGetPorts() {
+        Set<PortNumber> ports;
+
+        ports = config.getPorts(KEY1);
+        assertThat(ports.size(), is(2));
+        assertTrue(ports.contains(PORT3));
+        assertTrue(ports.contains(PORT4));
+
+        ports = config.getPorts(KEY2);
+        assertThat(ports.size(), is(2));
+        assertTrue(ports.contains(PORT3));
+        assertTrue(ports.contains(PORT4));
+
+        ports = config.getPorts(KEY3);
+        assertThat(ports.size(), is(2));
+        assertTrue(ports.contains(PORT4));
+        assertTrue(ports.contains(PORT5));
+    }
+
+    private class MockDelegate implements ConfigApplyDelegate {
+        @Override
+        public void onApply(Config config) {
+        }
+    }
+}
\ No newline at end of file
diff --git a/apps/segmentrouting/src/test/resources/sr-app-config-invalid.json b/apps/segmentrouting/src/test/resources/app-invalid.json
similarity index 100%
rename from apps/segmentrouting/src/test/resources/sr-app-config-invalid.json
rename to apps/segmentrouting/src/test/resources/app-invalid.json
diff --git a/apps/segmentrouting/src/test/resources/sr-app-config.json b/apps/segmentrouting/src/test/resources/app.json
similarity index 100%
rename from apps/segmentrouting/src/test/resources/sr-app-config.json
rename to apps/segmentrouting/src/test/resources/app.json
diff --git a/apps/segmentrouting/src/test/resources/sr-device-config.json b/apps/segmentrouting/src/test/resources/device.json
similarity index 100%
rename from apps/segmentrouting/src/test/resources/sr-device-config.json
rename to apps/segmentrouting/src/test/resources/device.json
diff --git a/apps/segmentrouting/src/test/resources/xconnect-invalid.json b/apps/segmentrouting/src/test/resources/xconnect-invalid.json
new file mode 100644
index 0000000..aa35f31
--- /dev/null
+++ b/apps/segmentrouting/src/test/resources/xconnect-invalid.json
@@ -0,0 +1,18 @@
+{
+  "of:0000000000000001": [
+    {
+      "vlan": 10,
+      "ports": [3, 4]
+    }
+  ],
+  "of:0000000000000002": [
+    {
+      "vlan": 10,
+      "ports": [3, 4]
+    },
+    {
+      "vlan": 20,
+      "ports": [4, 5, 6]
+    }
+  ]
+}
\ No newline at end of file
diff --git a/apps/segmentrouting/src/test/resources/xconnect.json b/apps/segmentrouting/src/test/resources/xconnect.json
new file mode 100644
index 0000000..ab9cfe6
--- /dev/null
+++ b/apps/segmentrouting/src/test/resources/xconnect.json
@@ -0,0 +1,19 @@
+{
+  "of:0000000000000001": [
+    {
+      "vlan": 10,
+      "ports": [3, 4],
+      "name": "OLT1"
+    }
+  ],
+  "of:0000000000000002": [
+    {
+      "vlan": 10,
+      "ports": [3, 4]
+    },
+    {
+      "vlan": 20,
+      "ports": [4, 5]
+    }
+  ]
+}
\ No newline at end of file
diff --git a/core/api/src/main/java/org/onosproject/net/flowobjective/ObjectiveError.java b/core/api/src/main/java/org/onosproject/net/flowobjective/ObjectiveError.java
index 8c20449..bd51902 100644
--- a/core/api/src/main/java/org/onosproject/net/flowobjective/ObjectiveError.java
+++ b/core/api/src/main/java/org/onosproject/net/flowobjective/ObjectiveError.java
@@ -34,11 +34,16 @@
     FLOWINSTALLATIONFAILED,
 
     /**
-     * THe group installation for this objective failed.
+     * The group installation for this objective failed.
      */
     GROUPINSTALLATIONFAILED,
 
     /**
+     * The group removal for this objective failed.
+     */
+    GROUPREMOVALFAILED,
+
+    /**
      * The group was reported as installed but is missing.
      */
     GROUPMISSING,
diff --git a/drivers/default/src/main/java/org/onosproject/driver/pipeline/Ofdpa2GroupHandler.java b/drivers/default/src/main/java/org/onosproject/driver/pipeline/Ofdpa2GroupHandler.java
index 6a2eb7e..80d8332 100644
--- a/drivers/default/src/main/java/org/onosproject/driver/pipeline/Ofdpa2GroupHandler.java
+++ b/drivers/default/src/main/java/org/onosproject/driver/pipeline/Ofdpa2GroupHandler.java
@@ -114,7 +114,8 @@
 
     protected DeviceId deviceId;
     private FlowObjectiveStore flowObjectiveStore;
-    private Cache<GroupKey, List<OfdpaNextGroup>> pendingNextObjectives;
+    private Cache<GroupKey, List<OfdpaNextGroup>> pendingAddNextObjectives;
+    private Cache<NextObjective, List<GroupKey>> pendingRemoveNextObjectives;
     private ConcurrentHashMap<GroupKey, Set<GroupChainElem>> pendingGroups;
     private ScheduledExecutorService groupChecker =
             Executors.newScheduledThreadPool(2, groupedThreads("onos/pipeliner", "ofdpa2-%d", log));
@@ -134,7 +135,7 @@
         this.storageService = serviceDirectory.get(StorageService.class);
         this.nextIndex = storageService.getAtomicCounter("group-id-index-counter");
 
-        pendingNextObjectives = CacheBuilder.newBuilder()
+        pendingAddNextObjectives = CacheBuilder.newBuilder()
                 .expireAfterWrite(20, TimeUnit.SECONDS)
                 .removalListener((
                         RemovalNotification<GroupKey, List<OfdpaNextGroup>> notification) -> {
@@ -142,7 +143,16 @@
                         notification.getValue().forEach(ofdpaNextGrp ->
                                 Ofdpa2Pipeline.fail(ofdpaNextGrp.nextObj,
                                         ObjectiveError.GROUPINSTALLATIONFAILED));
+                    }
+                }).build();
 
+        pendingRemoveNextObjectives = CacheBuilder.newBuilder()
+                .expireAfterWrite(20, TimeUnit.SECONDS)
+                .removalListener((
+                        RemovalNotification<NextObjective, List<GroupKey>> notification) -> {
+                    if (notification.getCause() == RemovalCause.EXPIRED) {
+                            Ofdpa2Pipeline.fail(notification.getKey(),
+                                    ObjectiveError.GROUPREMOVALFAILED);
                     }
                 }).build();
         pendingGroups = new ConcurrentHashMap<>();
@@ -1012,6 +1022,11 @@
      */
     protected void removeGroup(NextObjective nextObjective, NextGroup next) {
         List<Deque<GroupKey>> allgkeys = Ofdpa2Pipeline.appKryo.deserialize(next.data());
+
+        List<GroupKey> groupKeys = allgkeys.stream()
+                .map(Deque::getFirst).collect(Collectors.toList());
+        pendingRemoveNextObjectives.put(nextObjective, groupKeys);
+
         allgkeys.forEach(groupChain -> groupChain.forEach(groupKey ->
                 groupService.removeGroup(deviceId, groupKey, nextObjective.appId())));
         flowObjectiveStore.removeNextGroup(nextObjective.id());
@@ -1024,7 +1039,7 @@
     private void updatePendingNextObjective(GroupKey key, OfdpaNextGroup value) {
         List<OfdpaNextGroup> nextList = new CopyOnWriteArrayList<OfdpaNextGroup>();
         nextList.add(value);
-        List<OfdpaNextGroup> ret = pendingNextObjectives.asMap()
+        List<OfdpaNextGroup> ret = pendingAddNextObjectives.asMap()
                 .putIfAbsent(key, nextList);
         if (ret != null) {
             ret.add(value);
@@ -1079,13 +1094,13 @@
             Set<GroupKey> keys = pendingGroups.keySet().stream()
                     .filter(key -> groupService.getGroup(deviceId, key) != null)
                     .collect(Collectors.toSet());
-            Set<GroupKey> otherkeys = pendingNextObjectives.asMap().keySet().stream()
+            Set<GroupKey> otherkeys = pendingAddNextObjectives.asMap().keySet().stream()
                     .filter(otherkey -> groupService.getGroup(deviceId, otherkey) != null)
                     .collect(Collectors.toSet());
             keys.addAll(otherkeys);
 
             keys.stream().forEach(key ->
-                    processPendingGroupsOrNextObjectives(key, false));
+                    processPendingAddGroupsOrNextObjs(key, false));
         }
     }
 
@@ -1093,14 +1108,20 @@
         @Override
         public void event(GroupEvent event) {
             log.trace("received group event of type {}", event.type());
-            if (event.type() == GroupEvent.Type.GROUP_ADDED) {
-                GroupKey key = event.subject().appCookie();
-                processPendingGroupsOrNextObjectives(key, true);
+            switch (event.type()) {
+                case GROUP_ADDED:
+                    processPendingAddGroupsOrNextObjs(event.subject().appCookie(), true);
+                    break;
+                case GROUP_REMOVED:
+                    processPendingRemoveNextObjs(event.subject().appCookie());
+                    break;
+                default:
+                    break;
             }
         }
     }
 
-    private void processPendingGroupsOrNextObjectives(GroupKey key, boolean added) {
+    private void processPendingAddGroupsOrNextObjs(GroupKey key, boolean added) {
         //first check for group chain
         Set<GroupChainElem> gceSet = pendingGroups.remove(key);
         if (gceSet != null) {
@@ -1114,9 +1135,9 @@
             }
         } else {
             // otherwise chain complete - check for waiting nextObjectives
-            List<OfdpaNextGroup> nextGrpList = pendingNextObjectives.getIfPresent(key);
+            List<OfdpaNextGroup> nextGrpList = pendingAddNextObjectives.getIfPresent(key);
             if (nextGrpList != null) {
-                pendingNextObjectives.invalidate(key);
+                pendingAddNextObjectives.invalidate(key);
                 nextGrpList.forEach(nextGrp -> {
                     log.debug("Group service {} group key {} in device:{}. "
                                     + "Done implementing next objective: {} <<-->> gid:0x{}",
@@ -1137,6 +1158,17 @@
         }
     }
 
+    private void processPendingRemoveNextObjs(GroupKey key) {
+        pendingRemoveNextObjectives.asMap().forEach((nextObjective, groupKeys) -> {
+            if (groupKeys.isEmpty()) {
+                pendingRemoveNextObjectives.invalidate(nextObjective);
+                Ofdpa2Pipeline.pass(nextObjective);
+            } else {
+                groupKeys.remove(key);
+            }
+        });
+    }
+
     protected int getNextAvailableIndex() {
         return (int) nextIndex.incrementAndGet();
     }