lldp discovery independent of OF

Change-Id: I720f727f6628e30e5d732e6d7bf742d1b7050812
diff --git a/providers/lldp/pom.xml b/providers/lldp/pom.xml
new file mode 100644
index 0000000..06c18bd
--- /dev/null
+++ b/providers/lldp/pom.xml
@@ -0,0 +1,20 @@
+<?xml version="1.0" encoding="UTF-8"?>
+<project xmlns="http://maven.apache.org/POM/4.0.0"
+         xmlns:xsi="http://www.w3.org/2001/XMLSchema-instance"
+         xsi:schemaLocation="http://maven.apache.org/POM/4.0.0 http://maven.apache.org/maven-v4_0_0.xsd">
+    <modelVersion>4.0.0</modelVersion>
+
+
+    <parent>
+        <groupId>org.onlab.onos</groupId>
+        <artifactId>onos-providers</artifactId>
+        <version>1.0.0-SNAPSHOT</version>
+        <relativePath>../pom.xml</relativePath>
+    </parent>
+
+    <artifactId>onos-lldp-provider</artifactId>
+    <packaging>bundle</packaging>
+
+    <description>ONOS LLDP Link Discovery</description>
+
+</project>
diff --git a/providers/lldp/src/main/java/org/onlab/onos/provider/lldp/impl/LLDPLinkProvider.java b/providers/lldp/src/main/java/org/onlab/onos/provider/lldp/impl/LLDPLinkProvider.java
new file mode 100644
index 0000000..023940e
--- /dev/null
+++ b/providers/lldp/src/main/java/org/onlab/onos/provider/lldp/impl/LLDPLinkProvider.java
@@ -0,0 +1,165 @@
+package org.onlab.onos.provider.lldp.impl;
+
+import org.apache.felix.scr.annotations.Activate;
+import org.apache.felix.scr.annotations.Component;
+import org.apache.felix.scr.annotations.Deactivate;
+import org.apache.felix.scr.annotations.Reference;
+import org.apache.felix.scr.annotations.ReferenceCardinality;
+import org.onlab.onos.net.ConnectPoint;
+import org.onlab.onos.net.Device;
+import org.onlab.onos.net.DeviceId;
+import org.onlab.onos.net.device.DeviceEvent;
+import org.onlab.onos.net.device.DeviceListener;
+import org.onlab.onos.net.device.DeviceService;
+import org.onlab.onos.net.link.LinkProvider;
+import org.onlab.onos.net.link.LinkProviderRegistry;
+import org.onlab.onos.net.link.LinkProviderService;
+import org.onlab.onos.net.packet.PacketContext;
+import org.onlab.onos.net.packet.PacketProcessor;
+import org.onlab.onos.net.packet.PacketService;
+import org.onlab.onos.net.provider.AbstractProvider;
+import org.onlab.onos.net.provider.ProviderId;
+import org.slf4j.Logger;
+
+import java.util.Map;
+import java.util.concurrent.ConcurrentHashMap;
+
+import static org.slf4j.LoggerFactory.getLogger;
+
+
+/**
+ * Provider which uses an OpenFlow controller to detect network
+ * infrastructure links.
+ */
+@Component(immediate = true)
+public class LLDPLinkProvider extends AbstractProvider implements LinkProvider {
+
+    private final Logger log = getLogger(getClass());
+
+    @Reference(cardinality = ReferenceCardinality.MANDATORY_UNARY)
+    protected LinkProviderRegistry providerRegistry;
+
+    @Reference(cardinality = ReferenceCardinality.MANDATORY_UNARY)
+    protected DeviceService deviceService;
+
+    @Reference(cardinality = ReferenceCardinality.MANDATORY_UNARY)
+    protected PacketService packetSevice;
+
+    private LinkProviderService providerService;
+
+    private final boolean useBDDP = true;
+
+
+    private final InternalLinkProvider listener = new InternalLinkProvider();
+
+    protected final Map<DeviceId, LinkDiscovery> discoverers = new ConcurrentHashMap<>();
+
+    /**
+     * Creates an OpenFlow link provider.
+     */
+    public LLDPLinkProvider() {
+        super(new ProviderId("lldp", "org.onlab.onos.provider.lldp"));
+    }
+
+    @Activate
+    public void activate() {
+        providerService = providerRegistry.register(this);
+        deviceService.addListener(listener);
+        packetSevice.addProcessor(listener, 0);
+
+        log.info("Started");
+    }
+
+    @Deactivate
+    public void deactivate() {
+        for (LinkDiscovery ld : discoverers.values()) {
+            ld.stop();
+        }
+        providerRegistry.unregister(this);
+        deviceService.removeListener(listener);
+        packetSevice.removeProcessor(listener);
+        providerService = null;
+
+        log.info("Stopped");
+    }
+
+
+    private class InternalLinkProvider implements PacketProcessor, DeviceListener {
+
+        @Override
+        public void event(DeviceEvent event) {
+            LinkDiscovery ld = null;
+            Device device = event.subject();
+            // it's not a switch so leave.
+            if (device.type() != Device.Type.SWITCH) {
+                return;
+            }
+            switch (event.type()) {
+                case DEVICE_ADDED:
+                    discoverers.put(event.subject().id(),
+                                    new LinkDiscovery(event.subject(), packetSevice,
+                                                      providerService, useBDDP));
+                    break;
+                case PORT_ADDED:
+                case PORT_UPDATED:
+                    if (event.port().isEnabled()) {
+                        ld = discoverers.get(event.subject().id());
+                        if (ld == null) {
+                            return;
+                        }
+                        ld.addPort(event.port());
+                    } else {
+                        ConnectPoint point = new ConnectPoint(event.subject().id(),
+                                                              event.port().number());
+                        providerService.linksVanished(point);
+                    }
+                    break;
+                case PORT_REMOVED:
+                    ConnectPoint point = new ConnectPoint(event.subject().id(),
+                                                          event.port().number());
+                    providerService.linksVanished(point);
+                    break;
+                case DEVICE_REMOVED:
+                case DEVICE_SUSPENDED:
+                    ld = discoverers.get(event.subject().id());
+                    if (ld == null) {
+                        return;
+                    }
+                    ld.stop();
+                    providerService.linksVanished(event.subject().id());
+                    break;
+                case DEVICE_AVAILABILITY_CHANGED:
+                    ld = discoverers.get(event.subject().id());
+                    if (ld == null) {
+                        return;
+                    }
+                    if (deviceService.isAvailable(event.subject().id())) {
+                        ld.start();
+                    } else {
+                        providerService.linksVanished(event.subject().id());
+                        ld.stop();
+                    }
+                    break;
+                case DEVICE_UPDATED:
+                case DEVICE_MASTERSHIP_CHANGED:
+                    break;
+                default:
+                    log.debug("Unknown event {}", event);
+            }
+        }
+
+        @Override
+        public void process(PacketContext context) {
+            LinkDiscovery ld = discoverers.get(
+                    context.inPacket().receivedFrom().deviceId());
+            if (ld == null) {
+                return;
+            }
+
+            if (ld.handleLLDP(context)) {
+                context.block();
+            }
+        }
+    }
+
+}
diff --git a/providers/lldp/src/main/java/org/onlab/onos/provider/lldp/impl/LinkDiscovery.java b/providers/lldp/src/main/java/org/onlab/onos/provider/lldp/impl/LinkDiscovery.java
new file mode 100644
index 0000000..fc0a0f4
--- /dev/null
+++ b/providers/lldp/src/main/java/org/onlab/onos/provider/lldp/impl/LinkDiscovery.java
@@ -0,0 +1,349 @@
+/*******************************************************************************
+ * Copyright 2014 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.onlab.onos.provider.lldp.impl;
+
+
+import static org.slf4j.LoggerFactory.getLogger;
+
+import java.nio.ByteBuffer;
+import java.util.Collections;
+import java.util.HashMap;
+import java.util.HashSet;
+import java.util.Iterator;
+import java.util.Map;
+import java.util.Set;
+import java.util.concurrent.TimeUnit;
+import java.util.concurrent.atomic.AtomicInteger;
+
+import org.jboss.netty.util.Timeout;
+import org.jboss.netty.util.TimerTask;
+import org.onlab.onos.net.ConnectPoint;
+import org.onlab.onos.net.Device;
+import org.onlab.onos.net.DeviceId;
+import org.onlab.onos.net.Link.Type;
+import org.onlab.onos.net.Port;
+import org.onlab.onos.net.PortNumber;
+import org.onlab.onos.net.flow.DefaultTrafficTreatment;
+import org.onlab.onos.net.link.DefaultLinkDescription;
+import org.onlab.onos.net.link.LinkDescription;
+import org.onlab.onos.net.link.LinkProviderService;
+import org.onlab.onos.net.packet.DefaultOutboundPacket;
+import org.onlab.onos.net.packet.OutboundPacket;
+import org.onlab.onos.net.packet.PacketContext;
+import org.onlab.onos.net.packet.PacketService;
+import org.onlab.packet.Ethernet;
+import org.onlab.packet.ONOSLLDP;
+import org.onlab.util.Timer;
+import org.slf4j.Logger;
+
+
+
+/**
+ * Run discovery process from a physical switch. Ports are initially labeled as
+ * slow ports. When an LLDP is successfully received, label the remote port as
+ * fast. Every probeRate milliseconds, loop over all fast ports and send an
+ * LLDP, send an LLDP for a single slow port. Based on FlowVisor topology
+ * discovery implementation.
+ *
+ * TODO: add 'fast discovery' mode: drop LLDPs in destination switch but listen
+ * for flow_removed messages
+ */
+public class LinkDiscovery implements TimerTask {
+
+    private final Device device;
+    // send 1 probe every probeRate milliseconds
+    private final long probeRate;
+    private final Set<Long> slowPorts;
+    private final Set<Long> fastPorts;
+    // number of unacknowledged probes per port
+    private final Map<Long, AtomicInteger> portProbeCount;
+    // number of probes to send before link is removed
+    private static final short MAX_PROBE_COUNT = 3;
+    private final Logger log = getLogger(getClass());
+    private final ONOSLLDP lldpPacket;
+    private final Ethernet ethPacket;
+    private Ethernet bddpEth;
+    private final boolean useBDDP;
+    private final LinkProviderService linkProvider;
+    private final PacketService pktService;
+    private Timeout timeout;
+
+    /**
+     * Instantiates discovery manager for the given physical switch. Creates a
+     * generic LLDP packet that will be customized for the port it is sent out on.
+     * Starts the the timer for the discovery process.
+     *
+     * @param device the physical switch
+     * @param useBDDP flag to also use BDDP for discovery
+     */
+    public LinkDiscovery(Device device, PacketService pktService,
+             LinkProviderService providerService, Boolean... useBDDP) {
+        this.device = device;
+        this.probeRate = 3000;
+        this.linkProvider = providerService;
+        this.pktService = pktService;
+        this.slowPorts = Collections.synchronizedSet(new HashSet<Long>());
+        this.fastPorts = Collections.synchronizedSet(new HashSet<Long>());
+        this.portProbeCount = new HashMap<>();
+        this.lldpPacket = new ONOSLLDP();
+        this.lldpPacket.setChassisId(device.chassisId());
+        this.lldpPacket.setDevice(device.id().toString());
+
+
+        this.ethPacket = new Ethernet();
+        this.ethPacket.setEtherType(Ethernet.TYPE_LLDP);
+        this.ethPacket.setDestinationMACAddress(ONOSLLDP.LLDP_NICIRA);
+        this.ethPacket.setPayload(this.lldpPacket);
+        this.ethPacket.setPad(true);
+        this.useBDDP = useBDDP.length > 0 ? useBDDP[0] : false;
+        if (this.useBDDP) {
+            this.bddpEth = new Ethernet();
+            this.bddpEth.setPayload(this.lldpPacket);
+            this.bddpEth.setEtherType(Ethernet.TYPE_BSN);
+            this.bddpEth.setDestinationMACAddress(ONOSLLDP.BDDP_MULTICAST);
+            this.bddpEth.setPad(true);
+            log.info("Using BDDP to discover network");
+        }
+
+        start();
+        this.log.debug("Started discovery manager for switch {}",
+                       device.id());
+
+    }
+
+    /**
+     * Add physical port port to discovery process.
+     * Send out initial LLDP and label it as slow port.
+     *
+     * @param port the port
+     */
+    public void addPort(final Port port) {
+        this.log.debug("sending init probe to port {}",
+                           port.number().toLong());
+
+        sendProbes(port.number().toLong());
+
+        synchronized (this) {
+            this.slowPorts.add(port.number().toLong());
+        }
+
+
+    }
+
+    /**
+     * Removes physical port from discovery process.
+     *
+     * @param port the port
+     */
+    public void removePort(final Port port) {
+        // Ignore ports that are not on this switch
+
+        long portnum = port.number().toLong();
+        synchronized (this) {
+            if (this.slowPorts.contains(portnum)) {
+                this.slowPorts.remove(portnum);
+
+            } else if (this.fastPorts.contains(portnum)) {
+                this.fastPorts.remove(portnum);
+                this.portProbeCount.remove(portnum);
+                // no iterator to update
+            } else {
+                this.log.warn(
+                        "tried to dynamically remove non-existing port {}",
+                        portnum);
+            }
+        }
+    }
+
+    /**
+     * Method called by remote port to acknowledge receipt of LLDP sent by
+     * this port. If slow port, updates label to fast. If fast port, decrements
+     * number of unacknowledged probes.
+     *
+     * @param portNumber the port
+     */
+    public void ackProbe(final Long portNumber) {
+
+        synchronized (this) {
+            if (this.slowPorts.contains(portNumber)) {
+                this.log.debug("Setting slow port to fast: {}:{}",
+                        this.device.id(), portNumber);
+                this.slowPorts.remove(portNumber);
+                this.fastPorts.add(portNumber);
+                this.portProbeCount.put(portNumber, new AtomicInteger(0));
+            } else {
+                if (this.fastPorts.contains(portNumber)) {
+                    this.portProbeCount.get(portNumber).set(0);
+                } else {
+                    this.log.debug(
+                            "Got ackProbe for non-existing port: {}",
+                            portNumber);
+                }
+            }
+        }
+    }
+
+
+    /**
+     * Handles an incoming LLDP packet. Creates link in topology and sends ACK
+     * to port where LLDP originated.
+     */
+    public boolean handleLLDP(PacketContext context) {
+        Ethernet eth = context.inPacket().parsed();
+        ONOSLLDP onoslldp = ONOSLLDP.parseONOSLLDP(eth);
+        if (onoslldp != null) {
+            final PortNumber dstPort =
+                    context.inPacket().receivedFrom().port();
+            final PortNumber srcPort = PortNumber.portNumber(onoslldp.getPort());
+            final DeviceId srcDeviceId = DeviceId.deviceId(onoslldp.getDeviceString());
+            final DeviceId dstDeviceId = context.inPacket().receivedFrom().deviceId();
+            this.ackProbe(srcPort.toLong());
+            ConnectPoint src = new ConnectPoint(srcDeviceId, srcPort);
+            ConnectPoint dst = new ConnectPoint(dstDeviceId, dstPort);
+
+            LinkDescription ld;
+            if (eth.getEtherType() == Ethernet.TYPE_BSN) {
+                ld = new DefaultLinkDescription(src, dst, Type.INDIRECT);
+            } else {
+                ld = new DefaultLinkDescription(src, dst, Type.DIRECT);
+            }
+            linkProvider.linkDetected(ld);
+            return true;
+        }
+        return false;
+    }
+
+
+
+    /**
+     * Execute this method every t milliseconds. Loops over all ports
+     * labeled as fast and sends out an LLDP. Send out an LLDP on a single slow
+     * port.
+     *
+     * @param t timeout
+     * @throws Exception
+     */
+    @Override
+    public void run(final Timeout t) {
+        this.log.debug("sending probes");
+        synchronized (this) {
+            final Iterator<Long> fastIterator = this.fastPorts.iterator();
+            Long portNumber;
+            Integer probeCount;
+            while (fastIterator.hasNext()) {
+                portNumber = fastIterator.next();
+                probeCount = this.portProbeCount.get(portNumber)
+                        .getAndIncrement();
+
+                if (probeCount < LinkDiscovery.MAX_PROBE_COUNT) {
+                    this.log.debug("sending fast probe to port");
+                    sendProbes(portNumber);
+                } else {
+                    // Update fast and slow ports
+                    //fastIterator.remove();
+                    //this.slowPorts.add(portNumber);
+                    //this.portProbeCount.remove(portNumber);
+
+
+                    ConnectPoint cp = new ConnectPoint(
+                            device.id(),
+                            PortNumber.portNumber(portNumber));
+                    log.debug("Link down -> {}", cp);
+                    linkProvider.linksVanished(cp);
+                }
+            }
+
+            // send a probe for the next slow port
+            if (!this.slowPorts.isEmpty()) {
+                Iterator<Long> slowIterator = this.slowPorts.iterator();
+                while (slowIterator.hasNext()) {
+                    portNumber = slowIterator.next();
+                    this.log.debug("sending slow probe to port {}", portNumber);
+
+                    sendProbes(portNumber);
+
+                }
+            }
+        }
+
+        // reschedule timer
+        timeout = Timer.getTimer().newTimeout(this, this.probeRate,
+                TimeUnit.MILLISECONDS);
+    }
+
+    public void stop() {
+        timeout.cancel();
+    }
+
+    public void start() {
+        timeout = Timer.getTimer().newTimeout(this, 0,
+                                              TimeUnit.MILLISECONDS);
+    }
+
+    /**
+     * Creates packet_out LLDP for specified output port.
+     *
+     * @param port the port
+     * @return Packet_out message with LLDP data
+     */
+    private OutboundPacket createOutBoundLLDP(final Long port) {
+        if (port == null) {
+            return null;
+        }
+        this.lldpPacket.setPortId(port.intValue());
+        this.ethPacket.setSourceMACAddress("DE:AD:BE:EF:BA:11");
+
+        final byte[] lldp = this.ethPacket.serialize();
+        OutboundPacket outboundPacket = new DefaultOutboundPacket(
+                this.device.id(),
+                DefaultTrafficTreatment.builder().setOutput(
+                        PortNumber.portNumber(port)).build(),
+                ByteBuffer.wrap(lldp));
+        return outboundPacket;
+    }
+
+    /**
+     * Creates packet_out BDDP for specified output port.
+     *
+     * @param port the port
+     * @return Packet_out message with LLDP data
+     */
+    private OutboundPacket createOutBoundBDDP(final Long port) {
+        if (port == null) {
+            return null;
+        }
+        this.lldpPacket.setPortId(port.intValue());
+        this.bddpEth.setSourceMACAddress("DE:AD:BE:EF:BA:11");
+
+        final byte[] bddp = this.bddpEth.serialize();
+        OutboundPacket outboundPacket = new DefaultOutboundPacket(
+                this.device.id(),
+                DefaultTrafficTreatment.builder()
+                        .setOutput(PortNumber.portNumber(port)).build(),
+                ByteBuffer.wrap(bddp));
+        return outboundPacket;
+    }
+
+    private void sendProbes(Long portNumber) {
+        OutboundPacket pkt = this.createOutBoundLLDP(portNumber);
+        pktService.emit(pkt);
+        if (useBDDP) {
+            OutboundPacket bpkt = this.createOutBoundBDDP(portNumber);
+            pktService.emit(bpkt);
+        }
+    }
+
+}
diff --git a/providers/lldp/src/main/java/org/onlab/onos/provider/lldp/impl/package-info.java b/providers/lldp/src/main/java/org/onlab/onos/provider/lldp/impl/package-info.java
new file mode 100644
index 0000000..c7f5b61
--- /dev/null
+++ b/providers/lldp/src/main/java/org/onlab/onos/provider/lldp/impl/package-info.java
@@ -0,0 +1,4 @@
+/**
+ * Provider that uses the core as a means of infrastructure link inference.
+ */
+package org.onlab.onos.provider.lldp.impl;
diff --git a/providers/openflow/device/src/main/java/org/onlab/onos/provider/of/device/impl/OpenFlowDeviceProvider.java b/providers/openflow/device/src/main/java/org/onlab/onos/provider/of/device/impl/OpenFlowDeviceProvider.java
index 8958fb6..88cb1ac 100644
--- a/providers/openflow/device/src/main/java/org/onlab/onos/provider/of/device/impl/OpenFlowDeviceProvider.java
+++ b/providers/openflow/device/src/main/java/org/onlab/onos/provider/of/device/impl/OpenFlowDeviceProvider.java
@@ -23,6 +23,7 @@
 import org.onlab.onos.openflow.controller.OpenFlowSwitch;
 import org.onlab.onos.openflow.controller.OpenFlowSwitchListener;
 import org.onlab.onos.openflow.controller.RoleState;
+import org.onlab.packet.ChassisId;
 import org.projectfloodlight.openflow.protocol.OFPortConfig;
 import org.projectfloodlight.openflow.protocol.OFPortDesc;
 import org.projectfloodlight.openflow.protocol.OFPortState;
@@ -117,13 +118,14 @@
             }
             DeviceId did = deviceId(uri(dpid));
             OpenFlowSwitch sw = controller.getSwitch(dpid);
-
+            ChassisId cId = new ChassisId(dpid.value());
             DeviceDescription description =
                     new DefaultDeviceDescription(did.uri(), Device.Type.SWITCH,
                                                  sw.manfacturerDescription(),
                                                  sw.hardwareDescription(),
                                                  sw.softwareDescription(),
-                                                 sw.serialNumber());
+                                                 sw.serialNumber(),
+                                                cId);
             providerService.deviceConnected(did, description);
             providerService.updatePorts(did, buildPortDescriptions(sw.getPorts()));
         }
diff --git a/providers/openflow/device/src/test/java/org/onlab/onos/provider/of/device/impl/OpenFlowDeviceProviderTest.java b/providers/openflow/device/src/test/java/org/onlab/onos/provider/of/device/impl/OpenFlowDeviceProviderTest.java
index 8196cb8..2c45c5e 100644
--- a/providers/openflow/device/src/test/java/org/onlab/onos/provider/of/device/impl/OpenFlowDeviceProviderTest.java
+++ b/providers/openflow/device/src/test/java/org/onlab/onos/provider/of/device/impl/OpenFlowDeviceProviderTest.java
@@ -59,7 +59,7 @@
     private static final List<OFPortDesc> PLIST = Lists.newArrayList(PD1, PD2);
 
     private static final Device DEV1 =
-            new DefaultDevice(PID, DID1, SWITCH, "", "", "", "");
+            new DefaultDevice(PID, DID1, SWITCH, "", "", "", "", null);
 
     private static final TestOpenFlowSwitch SW1 = new TestOpenFlowSwitch();
 
diff --git a/providers/openflow/flow/src/main/java/org/onlab/onos/provider/of/flow/impl/OpenFlowRuleProvider.java b/providers/openflow/flow/src/main/java/org/onlab/onos/provider/of/flow/impl/OpenFlowRuleProvider.java
index ac0bb61..a815f69 100644
--- a/providers/openflow/flow/src/main/java/org/onlab/onos/provider/of/flow/impl/OpenFlowRuleProvider.java
+++ b/providers/openflow/flow/src/main/java/org/onlab/onos/provider/of/flow/impl/OpenFlowRuleProvider.java
@@ -1,20 +1,9 @@
 package org.onlab.onos.provider.of.flow.impl;
 
-import static org.slf4j.LoggerFactory.getLogger;
-
-import java.util.HashMap;
-import java.util.HashSet;
-import java.util.List;
-import java.util.Map;
-import java.util.Set;
-import java.util.concurrent.ConcurrentHashMap;
-import java.util.concurrent.CountDownLatch;
-import java.util.concurrent.ExecutionException;
-import java.util.concurrent.Future;
-import java.util.concurrent.TimeUnit;
-import java.util.concurrent.TimeoutException;
-import java.util.concurrent.atomic.AtomicBoolean;
-
+import com.google.common.collect.ArrayListMultimap;
+import com.google.common.collect.Lists;
+import com.google.common.collect.Maps;
+import com.google.common.collect.Multimap;
 import org.apache.felix.scr.annotations.Activate;
 import org.apache.felix.scr.annotations.Component;
 import org.apache.felix.scr.annotations.Deactivate;
@@ -68,10 +57,20 @@
 import org.projectfloodlight.openflow.types.U32;
 import org.slf4j.Logger;
 
-import com.google.common.collect.ArrayListMultimap;
-import com.google.common.collect.Lists;
-import com.google.common.collect.Maps;
-import com.google.common.collect.Multimap;
+import java.util.HashMap;
+import java.util.HashSet;
+import java.util.List;
+import java.util.Map;
+import java.util.Set;
+import java.util.concurrent.ConcurrentHashMap;
+import java.util.concurrent.CountDownLatch;
+import java.util.concurrent.ExecutionException;
+import java.util.concurrent.Future;
+import java.util.concurrent.TimeUnit;
+import java.util.concurrent.TimeoutException;
+import java.util.concurrent.atomic.AtomicBoolean;
+
+import static org.slf4j.LoggerFactory.getLogger;
 
 /**
  * Provider which uses an OpenFlow controller to detect network
@@ -166,6 +165,9 @@
         for (FlowRuleBatchEntry fbe : batch.getOperations()) {
             FlowRule flowRule = fbe.getTarget();
             OpenFlowSwitch sw = controller.getSwitch(Dpid.dpid(flowRule.deviceId().uri()));
+            if (sw == null) {
+                log.warn("WTF {}", flowRule.deviceId());
+            }
             sws.add(new Dpid(sw.getId()));
             FlowModBuilder builder = new FlowModBuilder(flowRule, sw.factory());
             switch (fbe.getOperator()) {
@@ -322,6 +324,7 @@
 
         public void fail(OFErrorMsg msg, Dpid dpid) {
             ok.set(false);
+            removeRequirement(dpid);
             FlowEntry fe = null;
             FlowRuleBatchEntry fbe = fms.get(msg.getXid());
             FlowRule offending = fbe.getTarget();
@@ -375,10 +378,7 @@
 
         public void satisfyRequirement(Dpid dpid) {
             log.warn("Satisfaction from switch {}", dpid);
-            sws.remove(dpid);
-            countDownLatch.countDown();
-            cleanUp();
-
+            removeRequirement(dpid);
         }
 
 
@@ -395,6 +395,7 @@
 
         @Override
         public boolean cancel(boolean mayInterruptIfRunning) {
+            ok.set(false);
             this.state = BatchState.CANCELLED;
             cleanUp();
             for (FlowRuleBatchEntry fbe : fms.values()) {
@@ -431,6 +432,7 @@
                 throws InterruptedException, ExecutionException,
                 TimeoutException {
             if (countDownLatch.await(timeout, unit)) {
+
                 this.state = BatchState.FINISHED;
                 return new CompletedBatchOperation(ok.get(), offendingFlowMods);
             }
@@ -438,7 +440,7 @@
         }
 
         private void cleanUp() {
-            if (sws.isEmpty()) {
+            if (isDone() || isCancelled()) {
                 pendingFutures.remove(pendingXid);
                 for (Long xid : fms.keySet()) {
                     pendingFMs.remove(xid);
@@ -446,6 +448,12 @@
             }
         }
 
+        private void removeRequirement(Dpid dpid) {
+            countDownLatch.countDown();
+            sws.remove(dpid);
+            cleanUp();
+        }
+
     }
 
 }
diff --git a/providers/openflow/packet/src/main/java/org/onlab/onos/provider/of/packet/impl/OpenFlowPacketProvider.java b/providers/openflow/packet/src/main/java/org/onlab/onos/provider/of/packet/impl/OpenFlowPacketProvider.java
index 94f7a33..8be1fec 100644
--- a/providers/openflow/packet/src/main/java/org/onlab/onos/provider/of/packet/impl/OpenFlowPacketProvider.java
+++ b/providers/openflow/packet/src/main/java/org/onlab/onos/provider/of/packet/impl/OpenFlowPacketProvider.java
@@ -28,7 +28,6 @@
 import org.onlab.onos.openflow.controller.OpenFlowPacketContext;
 import org.onlab.onos.openflow.controller.OpenFlowSwitch;
 import org.onlab.onos.openflow.controller.PacketListener;
-import org.onlab.packet.Ethernet;
 import org.projectfloodlight.openflow.protocol.OFPacketOut;
 import org.projectfloodlight.openflow.protocol.OFPortDesc;
 import org.projectfloodlight.openflow.protocol.action.OFAction;
@@ -96,13 +95,13 @@
             return;
         }
 
-        Ethernet eth = new Ethernet();
-        eth.deserialize(packet.data().array(), 0, packet.data().array().length);
+        //Ethernet eth = new Ethernet();
+        //eth.deserialize(packet.data().array(), 0, packet.data().array().length);
         OFPortDesc p = null;
         for (Instruction inst : packet.treatment().instructions()) {
             if (inst.type().equals(Instruction.Type.OUTPUT)) {
                 p = portDesc(((OutputInstruction) inst).port());
-                OFPacketOut po = packetOut(sw, eth, p.getPortNo());
+                OFPacketOut po = packetOut(sw, packet.data().array(), p.getPortNo());
                 sw.sendMsg(po);
             }
         }
@@ -116,7 +115,7 @@
         return builder.build();
     }
 
-    private OFPacketOut packetOut(OpenFlowSwitch sw, Ethernet eth, OFPort out) {
+    private OFPacketOut packetOut(OpenFlowSwitch sw, byte[] eth, OFPort out) {
         OFPacketOut.Builder builder = sw.factory().buildPacketOut();
         OFAction act = sw.factory().actions()
                 .buildOutput()
@@ -126,7 +125,7 @@
                 .setBufferId(OFBufferId.NO_BUFFER)
                 .setInPort(OFPort.NO_MASK)
                 .setActions(Collections.singletonList(act))
-                .setData(eth.serialize())
+                .setData(eth)
                 .build();
     }
 
diff --git a/providers/pom.xml b/providers/pom.xml
index da63b72..ff990de 100644
--- a/providers/pom.xml
+++ b/providers/pom.xml
@@ -18,6 +18,7 @@
 
     <modules>
         <module>openflow</module>
+        <module>lldp</module>
     </modules>
 
     <dependencies>