alshabib

Merge branch 'master' of ssh://gerrit.onlab.us:29418/onos-next

Showing 19 changed files with 595 additions and 41 deletions
......@@ -56,6 +56,13 @@ public interface MastershipService {
Set<DeviceId> getDevicesOf(NodeId nodeId);
/**
* Returns the mastership term service for getting term information.
*
* @return the MastershipTermService for this mastership manager
*/
MastershipTermService requestTermService();
/**
* Adds the specified mastership change listener.
*
* @param listener the mastership listener
......
......@@ -55,4 +55,13 @@ public interface MastershipStore extends Store<MastershipEvent, MastershipStoreD
* @return a mastership event
*/
MastershipEvent setMaster(NodeId nodeId, DeviceId deviceId);
/**
* Returns the current master and number of past mastership hand-offs
* (terms) for a device.
*
* @param deviceId the device identifier
* @return the current master's ID and the term value for device, or null
*/
MastershipTerm getTermFor(DeviceId deviceId);
}
......
package org.onlab.onos.cluster;
public class MastershipTerm {
private final NodeId master = null;
import java.util.Objects;
public final class MastershipTerm {
private final NodeId master;
private int termNumber;
private MastershipTerm(NodeId master, int term) {
this.master = master;
this.termNumber = term;
}
public static MastershipTerm of(NodeId master, int term) {
return new MastershipTerm(master, term);
}
public NodeId master() {
return master;
}
public int termNumber() {
return termNumber;
}
@Override
public int hashCode() {
return Objects.hash(master, termNumber);
}
@Override
public boolean equals(Object other) {
if (this == other) {
return true;
}
if (other instanceof MastershipTerm) {
MastershipTerm that = (MastershipTerm) other;
if (!this.master.equals(that.master)) {
return false;
}
if (this.termNumber != that.termNumber) {
return false;
}
return true;
}
return false;
}
}
......
package org.onlab.onos.net;
import java.util.Objects;
import com.google.common.base.MoreObjects;
// TODO Consider renaming.
// it's an identifier for a Link, but it's not ElementId, so not using LinkId.
/**
* Immutable representation of a link identity.
*/
public class LinkKey {
private final ConnectPoint src;
private final ConnectPoint dst;
/**
* Returns source connection point.
*
* @return source connection point
*/
public ConnectPoint src() {
return src;
}
/**
* Returns destination connection point.
*
* @return destination connection point
*/
public ConnectPoint dst() {
return dst;
}
/**
* Creates a link identifier with source and destination connection point.
*
* @param src source connection point
* @param dst destination connection point
*/
public LinkKey(ConnectPoint src, ConnectPoint dst) {
this.src = src;
this.dst = dst;
}
@Override
public int hashCode() {
return Objects.hash(src(), dst);
}
@Override
public boolean equals(Object obj) {
if (this == obj) {
return true;
}
if (obj instanceof LinkKey) {
final LinkKey other = (LinkKey) obj;
return Objects.equals(this.src(), other.src()) &&
Objects.equals(this.dst, other.dst);
}
return false;
}
@Override
public String toString() {
return MoreObjects.toStringHelper(getClass())
.add("src", src())
.add("dst", dst)
.toString();
}
}
......@@ -40,4 +40,9 @@ public class MastershipServiceAdapter implements MastershipService {
@Override
public void removeListener(MastershipListener listener) {
}
@Override
public MastershipTermService requestTermService() {
return null;
}
}
......
......@@ -12,6 +12,8 @@ import org.onlab.onos.cluster.MastershipEvent;
import org.onlab.onos.cluster.MastershipListener;
import org.onlab.onos.cluster.MastershipService;
import org.onlab.onos.cluster.MastershipStore;
import org.onlab.onos.cluster.MastershipTerm;
import org.onlab.onos.cluster.MastershipTermService;
import org.onlab.onos.cluster.NodeId;
import org.onlab.onos.event.AbstractListenerRegistry;
import org.onlab.onos.event.EventDeliveryService;
......@@ -103,6 +105,12 @@ public class MastershipManager
return store.getDevices(nodeId);
}
@Override
public MastershipTermService requestTermService() {
return new InternalMastershipTermService();
}
@Override
public void addListener(MastershipListener listener) {
checkNotNull(listener);
......@@ -124,4 +132,13 @@ public class MastershipManager
}
}
private class InternalMastershipTermService implements MastershipTermService {
@Override
public MastershipTerm getMastershipTerm(DeviceId deviceId) {
return store.getTermFor(deviceId);
}
}
}
......
......@@ -11,6 +11,7 @@ import org.onlab.onos.cluster.ControllerNode;
import org.onlab.onos.cluster.ControllerNode.State;
import org.onlab.onos.cluster.DefaultControllerNode;
import org.onlab.onos.cluster.MastershipService;
import org.onlab.onos.cluster.MastershipTermService;
import org.onlab.onos.cluster.NodeId;
import org.onlab.onos.event.impl.TestEventDispatcher;
import org.onlab.onos.net.DeviceId;
......@@ -100,6 +101,20 @@ public class MastershipManagerTest {
assertEquals("should be two devices:", 2, mgr.getDevicesOf(NID_LOCAL).size());
}
@Test
public void termService() {
MastershipTermService ts = mgr.requestTermService();
//term = 0 for both
mgr.setRole(NID_LOCAL, DEV_MASTER, MASTER);
assertEquals("inconsistent term: ", 0, ts.getMastershipTerm(DEV_MASTER).termNumber());
//hand devices to NID_LOCAL and back: term = 2
mgr.setRole(NID_OTHER, DEV_MASTER, MASTER);
mgr.setRole(NID_LOCAL, DEV_MASTER, MASTER);
assertEquals("inconsistent terms: ", 2, ts.getMastershipTerm(DEV_MASTER).termNumber());
}
private final class TestClusterService implements ClusterService {
ControllerNode local = new DefaultControllerNode(NID_LOCAL, LOCALHOST);
......
......@@ -15,6 +15,7 @@ import org.onlab.onos.cluster.ClusterService;
import org.onlab.onos.cluster.MastershipEvent;
import org.onlab.onos.cluster.MastershipStore;
import org.onlab.onos.cluster.MastershipStoreDelegate;
import org.onlab.onos.cluster.MastershipTerm;
import org.onlab.onos.cluster.NodeId;
import org.onlab.onos.net.DeviceId;
import org.onlab.onos.net.MastershipRole;
......@@ -115,4 +116,10 @@ public class DistributedMastershipStore
return nodeId.equals(master) ? MastershipRole.MASTER : MastershipRole.STANDBY;
}
@Override
public MastershipTerm getTermFor(DeviceId deviceId) {
// TODO Auto-generated method stub
return null;
}
}
......
......@@ -369,7 +369,7 @@ public class DistributedDeviceStore
}
@Override
protected void onUpdate(DeviceId deviceId, DefaultDevice device) {
protected void onUpdate(DeviceId deviceId, DefaultDevice oldDevice, DefaultDevice device) {
notifyDelegate(new DeviceEvent(DEVICE_UPDATED, device));
}
}
......@@ -390,7 +390,7 @@ public class DistributedDeviceStore
}
@Override
protected void onUpdate(DeviceId deviceId, Map<PortNumber, Port> ports) {
protected void onUpdate(DeviceId deviceId, Map<PortNumber, Port> oldPorts, Map<PortNumber, Port> ports) {
// notifyDelegate(new DeviceEvent(PORT_UPDATED, getDevice(deviceId)));
}
}
......
......@@ -101,7 +101,7 @@ public abstract class AbstractDistributedStore<E extends Event, D extends StoreD
V newVal = deserialize(event.getValue());
Optional<V> newValue = Optional.of(newVal);
cache.asMap().replace(key, oldValue, newValue);
onUpdate(key, newVal);
onUpdate(key, oldVal, newVal);
}
@Override
......@@ -125,9 +125,10 @@ public abstract class AbstractDistributedStore<E extends Event, D extends StoreD
* Cache entry update hook.
*
* @param key new key
* @param oldValue old value
* @param newVal new value
*/
protected void onUpdate(K key, V newVal) {
protected void onUpdate(K key, V oldValue, V newVal) {
}
/**
......
......@@ -14,19 +14,26 @@ import org.apache.felix.scr.annotations.Service;
import org.onlab.onos.cluster.ControllerNode;
import org.onlab.onos.cluster.DefaultControllerNode;
import org.onlab.onos.cluster.NodeId;
import org.onlab.onos.net.ConnectPoint;
import org.onlab.onos.net.DefaultDevice;
import org.onlab.onos.net.DefaultLink;
import org.onlab.onos.net.DefaultPort;
import org.onlab.onos.net.Device;
import org.onlab.onos.net.DeviceId;
import org.onlab.onos.net.Element;
import org.onlab.onos.net.Link;
import org.onlab.onos.net.LinkKey;
import org.onlab.onos.net.MastershipRole;
import org.onlab.onos.net.Port;
import org.onlab.onos.net.PortNumber;
import org.onlab.onos.net.provider.ProviderId;
import org.onlab.onos.store.common.StoreService;
import org.onlab.onos.store.serializers.ConnectPointSerializer;
import org.onlab.onos.store.serializers.DefaultLinkSerializer;
import org.onlab.onos.store.serializers.DefaultPortSerializer;
import org.onlab.onos.store.serializers.DeviceIdSerializer;
import org.onlab.onos.store.serializers.IpPrefixSerializer;
import org.onlab.onos.store.serializers.LinkKeySerializer;
import org.onlab.onos.store.serializers.NodeIdSerializer;
import org.onlab.onos.store.serializers.OnosTimestampSerializer;
import org.onlab.onos.store.serializers.PortNumberSerializer;
......@@ -84,7 +91,9 @@ public class StoreManager implements StoreService {
DefaultDevice.class,
MastershipRole.class,
Port.class,
Element.class
Element.class,
Link.Type.class
)
.register(IpPrefix.class, new IpPrefixSerializer())
.register(URI.class, new URISerializer())
......@@ -94,6 +103,9 @@ public class StoreManager implements StoreService {
.register(PortNumber.class, new PortNumberSerializer())
.register(DefaultPort.class, new DefaultPortSerializer())
.register(OnosTimestamp.class, new OnosTimestampSerializer())
.register(LinkKey.class, new LinkKeySerializer())
.register(ConnectPoint.class, new ConnectPointSerializer())
.register(DefaultLink.class, new DefaultLinkSerializer())
.build()
.populate(10);
}
......
package org.onlab.onos.store.link.impl;
import static com.google.common.cache.CacheBuilder.newBuilder;
import static org.onlab.onos.net.Link.Type.DIRECT;
import static org.onlab.onos.net.Link.Type.INDIRECT;
import static org.onlab.onos.net.link.LinkEvent.Type.LINK_ADDED;
import static org.onlab.onos.net.link.LinkEvent.Type.LINK_REMOVED;
import static org.onlab.onos.net.link.LinkEvent.Type.LINK_UPDATED;
import static org.slf4j.LoggerFactory.getLogger;
import java.util.HashSet;
import java.util.Set;
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.Service;
import org.onlab.onos.net.ConnectPoint;
import org.onlab.onos.net.DefaultLink;
import org.onlab.onos.net.DeviceId;
import org.onlab.onos.net.Link;
import org.onlab.onos.net.LinkKey;
import org.onlab.onos.net.link.LinkDescription;
import org.onlab.onos.net.link.LinkEvent;
import org.onlab.onos.net.link.LinkStore;
import org.onlab.onos.net.link.LinkStoreDelegate;
import org.onlab.onos.net.provider.ProviderId;
import org.onlab.onos.store.impl.AbsentInvalidatingLoadingCache;
import org.onlab.onos.store.impl.AbstractDistributedStore;
import org.onlab.onos.store.impl.OptionalCacheLoader;
import org.slf4j.Logger;
import com.google.common.base.Optional;
import com.google.common.cache.LoadingCache;
import com.google.common.collect.HashMultimap;
import com.google.common.collect.ImmutableSet;
import com.google.common.collect.Multimap;
import com.google.common.collect.ImmutableSet.Builder;
import com.hazelcast.core.IMap;
/**
* Manages inventory of infrastructure links using Hazelcast-backed map.
*/
@Component(immediate = true)
@Service
public class DistributedLinkStore
extends AbstractDistributedStore<LinkEvent, LinkStoreDelegate>
implements LinkStore {
private final Logger log = getLogger(getClass());
// Link inventory
private IMap<byte[], byte[]> rawLinks;
private LoadingCache<LinkKey, Optional<DefaultLink>> links;
// TODO synchronize?
// Egress and ingress link sets
private final Multimap<DeviceId, Link> srcLinks = HashMultimap.create();
private final Multimap<DeviceId, Link> dstLinks = HashMultimap.create();
@Override
@Activate
public void activate() {
super.activate();
boolean includeValue = true;
// TODO decide on Map name scheme to avoid collision
rawLinks = theInstance.getMap("links");
final OptionalCacheLoader<LinkKey, DefaultLink> linkLoader
= new OptionalCacheLoader<>(storeService, rawLinks);
links = new AbsentInvalidatingLoadingCache<>(newBuilder().build(linkLoader));
// refresh/populate cache based on notification from other instance
rawLinks.addEntryListener(new RemoteLinkEventHandler(links), includeValue);
loadLinkCache();
log.info("Started");
}
@Deactivate
public void deactivate() {
super.activate();
log.info("Stopped");
}
private void loadLinkCache() {
for (byte[] keyBytes : rawLinks.keySet()) {
final LinkKey id = deserialize(keyBytes);
links.refresh(id);
}
}
@Override
public int getLinkCount() {
return links.asMap().size();
}
@Override
public Iterable<Link> getLinks() {
Builder<Link> builder = ImmutableSet.builder();
for (Optional<DefaultLink> e : links.asMap().values()) {
if (e.isPresent()) {
builder.add(e.get());
}
}
return builder.build();
}
@Override
public Set<Link> getDeviceEgressLinks(DeviceId deviceId) {
return ImmutableSet.copyOf(srcLinks.get(deviceId));
}
@Override
public Set<Link> getDeviceIngressLinks(DeviceId deviceId) {
return ImmutableSet.copyOf(dstLinks.get(deviceId));
}
@Override
public Link getLink(ConnectPoint src, ConnectPoint dst) {
return links.getUnchecked(new LinkKey(src, dst)).orNull();
}
@Override
public Set<Link> getEgressLinks(ConnectPoint src) {
Set<Link> egress = new HashSet<>();
for (Link link : srcLinks.get(src.deviceId())) {
if (link.src().equals(src)) {
egress.add(link);
}
}
return egress;
}
@Override
public Set<Link> getIngressLinks(ConnectPoint dst) {
Set<Link> ingress = new HashSet<>();
for (Link link : dstLinks.get(dst.deviceId())) {
if (link.dst().equals(dst)) {
ingress.add(link);
}
}
return ingress;
}
@Override
public LinkEvent createOrUpdateLink(ProviderId providerId,
LinkDescription linkDescription) {
LinkKey key = new LinkKey(linkDescription.src(), linkDescription.dst());
Optional<DefaultLink> link = links.getUnchecked(key);
if (!link.isPresent()) {
return createLink(providerId, key, linkDescription);
}
return updateLink(providerId, link.get(), key, linkDescription);
}
// Creates and stores the link and returns the appropriate event.
private LinkEvent createLink(ProviderId providerId, LinkKey key,
LinkDescription linkDescription) {
DefaultLink link = new DefaultLink(providerId, key.src(), key.dst(),
linkDescription.type());
synchronized (this) {
final byte[] keyBytes = serialize(key);
rawLinks.put(keyBytes, serialize(link));
links.asMap().putIfAbsent(key, Optional.of(link));
addNewLink(link);
}
return new LinkEvent(LINK_ADDED, link);
}
// update Egress and ingress link sets
private void addNewLink(DefaultLink link) {
synchronized (this) {
srcLinks.put(link.src().deviceId(), link);
dstLinks.put(link.dst().deviceId(), link);
}
}
// Updates, if necessary the specified link and returns the appropriate event.
private LinkEvent updateLink(ProviderId providerId, DefaultLink link,
LinkKey key, LinkDescription linkDescription) {
// FIXME confirm Link update condition is OK
if (link.type() == INDIRECT && linkDescription.type() == DIRECT) {
synchronized (this) {
DefaultLink updated =
new DefaultLink(providerId, link.src(), link.dst(),
linkDescription.type());
final byte[] keyBytes = serialize(key);
rawLinks.put(keyBytes, serialize(updated));
links.asMap().replace(key, Optional.of(link), Optional.of(updated));
replaceLink(link, updated);
return new LinkEvent(LINK_UPDATED, updated);
}
}
return null;
}
// update Egress and ingress link sets
private void replaceLink(DefaultLink link, DefaultLink updated) {
synchronized (this) {
srcLinks.remove(link.src().deviceId(), link);
dstLinks.remove(link.dst().deviceId(), link);
srcLinks.put(link.src().deviceId(), updated);
dstLinks.put(link.dst().deviceId(), updated);
}
}
@Override
public LinkEvent removeLink(ConnectPoint src, ConnectPoint dst) {
synchronized (this) {
LinkKey key = new LinkKey(src, dst);
byte[] keyBytes = serialize(key);
Link link = deserialize(rawLinks.remove(keyBytes));
links.invalidate(key);
if (link != null) {
removeLink(link);
return new LinkEvent(LINK_REMOVED, link);
}
return null;
}
}
// update Egress and ingress link sets
private void removeLink(Link link) {
synchronized (this) {
srcLinks.remove(link.src().deviceId(), link);
dstLinks.remove(link.dst().deviceId(), link);
}
}
private class RemoteLinkEventHandler extends RemoteEventHandler<LinkKey, DefaultLink> {
public RemoteLinkEventHandler(LoadingCache<LinkKey, Optional<DefaultLink>> cache) {
super(cache);
}
@Override
protected void onAdd(LinkKey key, DefaultLink newVal) {
addNewLink(newVal);
notifyDelegate(new LinkEvent(LINK_ADDED, newVal));
}
@Override
protected void onUpdate(LinkKey key, DefaultLink oldVal, DefaultLink newVal) {
replaceLink(oldVal, newVal);
notifyDelegate(new LinkEvent(LINK_UPDATED, newVal));
}
@Override
protected void onRemove(LinkKey key, DefaultLink val) {
removeLink(val);
notifyDelegate(new LinkEvent(LINK_REMOVED, val));
}
}
}
/**
* Implementation of link store using Hazelcast distributed structures.
*/
package org.onlab.onos.store.link.impl;
package org.onlab.onos.store.serializers;
import org.onlab.onos.net.ConnectPoint;
import org.onlab.onos.net.ElementId;
import org.onlab.onos.net.PortNumber;
import com.esotericsoftware.kryo.Kryo;
import com.esotericsoftware.kryo.Serializer;
import com.esotericsoftware.kryo.io.Input;
import com.esotericsoftware.kryo.io.Output;
/**
* Kryo Serializer for {@link ConnectPointSerializer}.
*/
public class ConnectPointSerializer extends Serializer<ConnectPoint> {
/**
* Default constructor.
*/
public ConnectPointSerializer() {
// non-null, immutable
super(false, true);
}
@Override
public void write(Kryo kryo, Output output, ConnectPoint object) {
kryo.writeClassAndObject(output, object.elementId());
kryo.writeClassAndObject(output, object.port());
}
@Override
public ConnectPoint read(Kryo kryo, Input input, Class<ConnectPoint> type) {
ElementId elementId = (ElementId) kryo.readClassAndObject(input);
PortNumber portNumber = (PortNumber) kryo.readClassAndObject(input);
return new ConnectPoint(elementId, portNumber);
}
}
package org.onlab.onos.store.serializers;
import org.onlab.onos.net.ConnectPoint;
import org.onlab.onos.net.DefaultLink;
import org.onlab.onos.net.Link.Type;
import org.onlab.onos.net.provider.ProviderId;
import com.esotericsoftware.kryo.Kryo;
import com.esotericsoftware.kryo.Serializer;
import com.esotericsoftware.kryo.io.Input;
import com.esotericsoftware.kryo.io.Output;
/**
* Kryo Serializer for {@link DefaultLink}.
*/
public class DefaultLinkSerializer extends Serializer<DefaultLink> {
/**
* Default constructor.
*/
public DefaultLinkSerializer() {
// non-null, immutable
super(false, true);
}
@Override
public void write(Kryo kryo, Output output, DefaultLink object) {
kryo.writeClassAndObject(output, object.providerId());
kryo.writeClassAndObject(output, object.src());
kryo.writeClassAndObject(output, object.dst());
kryo.writeClassAndObject(output, object.type());
}
@Override
public DefaultLink read(Kryo kryo, Input input, Class<DefaultLink> type) {
ProviderId providerId = (ProviderId) kryo.readClassAndObject(input);
ConnectPoint src = (ConnectPoint) kryo.readClassAndObject(input);
ConnectPoint dst = (ConnectPoint) kryo.readClassAndObject(input);
Type linkType = (Type) kryo.readClassAndObject(input);
return new DefaultLink(providerId, src, dst, linkType);
}
}
package org.onlab.onos.store.serializers;
import org.onlab.onos.net.ConnectPoint;
import org.onlab.onos.net.LinkKey;
import com.esotericsoftware.kryo.Kryo;
import com.esotericsoftware.kryo.Serializer;
import com.esotericsoftware.kryo.io.Input;
import com.esotericsoftware.kryo.io.Output;
/**
* Kryo Serializer for {@link LinkKey}.
*/
public class LinkKeySerializer extends Serializer<LinkKey> {
/**
* Default constructor.
*/
public LinkKeySerializer() {
// non-null, immutable
super(false, true);
}
@Override
public void write(Kryo kryo, Output output, LinkKey object) {
kryo.writeClassAndObject(output, object.src());
kryo.writeClassAndObject(output, object.dst());
}
@Override
public LinkKey read(Kryo kryo, Input input, Class<LinkKey> type) {
ConnectPoint src = (ConnectPoint) kryo.readClassAndObject(input);
ConnectPoint dst = (ConnectPoint) kryo.readClassAndObject(input);
return new LinkKey(src, dst);
}
}
......@@ -3,6 +3,7 @@ package org.onlab.onos.net.trivial.impl;
import com.google.common.collect.HashMultimap;
import com.google.common.collect.ImmutableSet;
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;
......@@ -11,6 +12,7 @@ import org.onlab.onos.net.ConnectPoint;
import org.onlab.onos.net.DefaultLink;
import org.onlab.onos.net.DeviceId;
import org.onlab.onos.net.Link;
import org.onlab.onos.net.LinkKey;
import org.onlab.onos.net.link.LinkDescription;
import org.onlab.onos.net.link.LinkEvent;
import org.onlab.onos.net.link.LinkStore;
......@@ -22,7 +24,6 @@ import org.slf4j.Logger;
import java.util.Collections;
import java.util.HashSet;
import java.util.Map;
import java.util.Objects;
import java.util.Set;
import java.util.concurrent.ConcurrentHashMap;
......@@ -123,7 +124,7 @@ public class SimpleLinkStore
// Creates and stores the link and returns the appropriate event.
private LinkEvent createLink(ProviderId providerId, LinkKey key,
LinkDescription linkDescription) {
DefaultLink link = new DefaultLink(providerId, key.src, key.dst,
DefaultLink link = new DefaultLink(providerId, key.src(), key.dst(),
linkDescription.type());
synchronized (this) {
links.put(key, link);
......@@ -165,33 +166,4 @@ public class SimpleLinkStore
return null;
}
}
// Auxiliary key to track links.
private class LinkKey {
final ConnectPoint src;
final ConnectPoint dst;
LinkKey(ConnectPoint src, ConnectPoint dst) {
this.src = src;
this.dst = dst;
}
@Override
public int hashCode() {
return Objects.hash(src, dst);
}
@Override
public boolean equals(Object obj) {
if (this == obj) {
return true;
}
if (obj instanceof LinkKey) {
final LinkKey other = (LinkKey) obj;
return Objects.equals(this.src, other.src) &&
Objects.equals(this.dst, other.dst);
}
return false;
}
}
}
......
......@@ -3,11 +3,13 @@ package org.onlab.onos.net.trivial.impl;
import static org.slf4j.LoggerFactory.getLogger;
import java.util.Collections;
import java.util.HashMap;
import java.util.HashSet;
import java.util.Map;
import java.util.Set;
import java.util.concurrent.ConcurrentHashMap;
import java.util.concurrent.ConcurrentMap;
import java.util.concurrent.atomic.AtomicInteger;
import org.apache.felix.scr.annotations.Activate;
import org.apache.felix.scr.annotations.Component;
......@@ -18,6 +20,7 @@ import org.onlab.onos.cluster.DefaultControllerNode;
import org.onlab.onos.cluster.MastershipEvent;
import org.onlab.onos.cluster.MastershipStore;
import org.onlab.onos.cluster.MastershipStoreDelegate;
import org.onlab.onos.cluster.MastershipTerm;
import org.onlab.onos.cluster.NodeId;
import org.onlab.onos.net.DeviceId;
import org.onlab.onos.net.MastershipRole;
......@@ -47,6 +50,7 @@ public class SimpleMastershipStore
//devices mapped to their masters, to emulate multiple nodes
protected final ConcurrentMap<DeviceId, NodeId> masterMap =
new ConcurrentHashMap<>();
protected final Map<DeviceId, AtomicInteger> termMap = new HashMap<>();
@Activate
public void activate() {
......@@ -63,15 +67,21 @@ public class SimpleMastershipStore
NodeId node = masterMap.get(deviceId);
if (node == null) {
masterMap.put(deviceId, nodeId);
synchronized (this) {
masterMap.put(deviceId, nodeId);
termMap.put(deviceId, new AtomicInteger());
}
return new MastershipEvent(MASTER_CHANGED, deviceId, nodeId);
}
if (node.equals(nodeId)) {
return null;
} else {
masterMap.put(deviceId, nodeId);
return new MastershipEvent(MASTER_CHANGED, deviceId, nodeId);
synchronized (this) {
masterMap.put(deviceId, nodeId);
termMap.get(deviceId).incrementAndGet();
return new MastershipEvent(MASTER_CHANGED, deviceId, nodeId);
}
}
}
......@@ -114,4 +124,13 @@ public class SimpleMastershipStore
return role;
}
@Override
public MastershipTerm getTermFor(DeviceId deviceId) {
if (masterMap.get(deviceId) == null) {
return null;
}
return MastershipTerm.of(
masterMap.get(deviceId), termMap.get(deviceId).get());
}
}
......