Showing
19 changed files
with
884 additions
and
118 deletions
core/store/dist/src/main/java/org/onlab/onos/store/cluster/impl/ClusterConnectionListener.java
0 → 100644
| 1 | +package org.onlab.onos.store.cluster.impl; | ||
| 2 | + | ||
| 3 | +import org.onlab.nio.AcceptorLoop; | ||
| 4 | +import org.onlab.packet.IpPrefix; | ||
| 5 | + | ||
| 6 | +import java.io.IOException; | ||
| 7 | +import java.net.InetSocketAddress; | ||
| 8 | +import java.net.Socket; | ||
| 9 | +import java.nio.channels.ServerSocketChannel; | ||
| 10 | +import java.nio.channels.SocketChannel; | ||
| 11 | + | ||
| 12 | +import static java.net.InetAddress.getByAddress; | ||
| 13 | + | ||
| 14 | +/** | ||
| 15 | + * Listens to inbound connection requests and accepts them. | ||
| 16 | + */ | ||
| 17 | +public class ClusterConnectionListener extends AcceptorLoop { | ||
| 18 | + | ||
| 19 | + private static final long SELECT_TIMEOUT = 50; | ||
| 20 | + private static final int COMM_BUFFER_SIZE = 32 * 1024; | ||
| 21 | + | ||
| 22 | + private static final boolean SO_NO_DELAY = false; | ||
| 23 | + private static final int SO_SEND_BUFFER_SIZE = COMM_BUFFER_SIZE; | ||
| 24 | + private static final int SO_RCV_BUFFER_SIZE = COMM_BUFFER_SIZE; | ||
| 25 | + | ||
| 26 | + private final WorkerFinder workerFinder; | ||
| 27 | + | ||
| 28 | + ClusterConnectionListener(IpPrefix ip, int tcpPort, | ||
| 29 | + WorkerFinder workerFinder) throws IOException { | ||
| 30 | + super(SELECT_TIMEOUT, new InetSocketAddress(getByAddress(ip.toOctets()), tcpPort)); | ||
| 31 | + this.workerFinder = workerFinder; | ||
| 32 | + } | ||
| 33 | + | ||
| 34 | + @Override | ||
| 35 | + protected void acceptConnection(ServerSocketChannel channel) throws IOException { | ||
| 36 | + SocketChannel sc = channel.accept(); | ||
| 37 | + sc.configureBlocking(false); | ||
| 38 | + | ||
| 39 | + Socket so = sc.socket(); | ||
| 40 | + so.setTcpNoDelay(SO_NO_DELAY); | ||
| 41 | + so.setReceiveBufferSize(SO_RCV_BUFFER_SIZE); | ||
| 42 | + so.setSendBufferSize(SO_SEND_BUFFER_SIZE); | ||
| 43 | + | ||
| 44 | + workerFinder.findWorker().acceptStream(sc); | ||
| 45 | + } | ||
| 46 | + | ||
| 47 | +} |
| 1 | +package org.onlab.onos.store.cluster.impl; | ||
| 2 | + | ||
| 3 | +import org.onlab.nio.IOLoop; | ||
| 4 | +import org.onlab.nio.MessageStream; | ||
| 5 | +import org.onlab.onos.cluster.DefaultControllerNode; | ||
| 6 | +import org.onlab.onos.store.cluster.messaging.ClusterMessage; | ||
| 7 | +import org.onlab.onos.store.cluster.messaging.ClusterMessageStream; | ||
| 8 | +import org.onlab.onos.store.cluster.messaging.SerializationService; | ||
| 9 | +import org.slf4j.Logger; | ||
| 10 | +import org.slf4j.LoggerFactory; | ||
| 11 | + | ||
| 12 | +import java.io.IOException; | ||
| 13 | +import java.net.InetSocketAddress; | ||
| 14 | +import java.nio.channels.ByteChannel; | ||
| 15 | +import java.nio.channels.SelectionKey; | ||
| 16 | +import java.nio.channels.SocketChannel; | ||
| 17 | +import java.util.List; | ||
| 18 | +import java.util.Objects; | ||
| 19 | + | ||
| 20 | +import static org.onlab.packet.IpPrefix.valueOf; | ||
| 21 | + | ||
| 22 | +/** | ||
| 23 | + * Performs the IO operations related to a cluster-wide communications. | ||
| 24 | + */ | ||
| 25 | +public class ClusterIOWorker extends | ||
| 26 | + IOLoop<ClusterMessage, ClusterMessageStream> { | ||
| 27 | + | ||
| 28 | + private final Logger log = LoggerFactory.getLogger(getClass()); | ||
| 29 | + | ||
| 30 | + private static final long SELECT_TIMEOUT = 50; | ||
| 31 | + | ||
| 32 | + private final ConnectionManager connectionManager; | ||
| 33 | + private final CommunicationsDelegate commsDelegate; | ||
| 34 | + private final SerializationService serializationService; | ||
| 35 | + private final ClusterMessage helloMessage; | ||
| 36 | + | ||
| 37 | + /** | ||
| 38 | + * Creates a new cluster IO worker. | ||
| 39 | + * | ||
| 40 | + * @param connectionManager parent connection manager | ||
| 41 | + * @param commsDelegate communications delegate for dispatching | ||
| 42 | + * @param serializationService serialization service for encode/decode | ||
| 43 | + * @param helloMessage hello message for greeting peers | ||
| 44 | + * @throws IOException if errors occur during IO loop ignition | ||
| 45 | + */ | ||
| 46 | + ClusterIOWorker(ConnectionManager connectionManager, | ||
| 47 | + CommunicationsDelegate commsDelegate, | ||
| 48 | + SerializationService serializationService, | ||
| 49 | + ClusterMessage helloMessage) throws IOException { | ||
| 50 | + super(SELECT_TIMEOUT); | ||
| 51 | + this.connectionManager = connectionManager; | ||
| 52 | + this.commsDelegate = commsDelegate; | ||
| 53 | + this.serializationService = serializationService; | ||
| 54 | + this.helloMessage = helloMessage; | ||
| 55 | + } | ||
| 56 | + | ||
| 57 | + @Override | ||
| 58 | + protected ClusterMessageStream createStream(ByteChannel byteChannel) { | ||
| 59 | + return new ClusterMessageStream(serializationService, this, byteChannel); | ||
| 60 | + } | ||
| 61 | + | ||
| 62 | + @Override | ||
| 63 | + protected void processMessages(List<ClusterMessage> messages, MessageStream<ClusterMessage> stream) { | ||
| 64 | + for (ClusterMessage message : messages) { | ||
| 65 | + commsDelegate.dispatch(message); | ||
| 66 | + } | ||
| 67 | + } | ||
| 68 | + | ||
| 69 | + @Override | ||
| 70 | + public ClusterMessageStream acceptStream(SocketChannel channel) { | ||
| 71 | + ClusterMessageStream stream = super.acceptStream(channel); | ||
| 72 | + try { | ||
| 73 | + InetSocketAddress sa = (InetSocketAddress) channel.getRemoteAddress(); | ||
| 74 | + log.info("Accepted connection from node {}", valueOf(sa.getAddress().getAddress())); | ||
| 75 | + stream.write(helloMessage); | ||
| 76 | + | ||
| 77 | + } catch (IOException e) { | ||
| 78 | + log.warn("Unable to accept connection from an unknown end-point", e); | ||
| 79 | + } | ||
| 80 | + return stream; | ||
| 81 | + } | ||
| 82 | + | ||
| 83 | + @Override | ||
| 84 | + protected void connect(SelectionKey key) throws IOException { | ||
| 85 | + try { | ||
| 86 | + super.connect(key); | ||
| 87 | + ClusterMessageStream stream = (ClusterMessageStream) key.attachment(); | ||
| 88 | + stream.write(helloMessage); | ||
| 89 | + | ||
| 90 | + } catch (IOException e) { | ||
| 91 | + if (!Objects.equals(e.getMessage(), "Connection refused")) { | ||
| 92 | + throw e; | ||
| 93 | + } | ||
| 94 | + } | ||
| 95 | + } | ||
| 96 | + | ||
| 97 | + @Override | ||
| 98 | + protected void removeStream(MessageStream<ClusterMessage> stream) { | ||
| 99 | + DefaultControllerNode node = ((ClusterMessageStream) stream).node(); | ||
| 100 | + if (node != null) { | ||
| 101 | + log.info("Closed connection to node {}", node.id()); | ||
| 102 | + connectionManager.removeNodeStream(node); | ||
| 103 | + } | ||
| 104 | + super.removeStream(stream); | ||
| 105 | + } | ||
| 106 | + | ||
| 107 | +} |
core/store/dist/src/main/java/org/onlab/onos/store/cluster/impl/ClusterNodesDelegate.java
0 → 100644
| 1 | +package org.onlab.onos.store.cluster.impl; | ||
| 2 | + | ||
| 3 | +import org.onlab.onos.cluster.DefaultControllerNode; | ||
| 4 | + | ||
| 5 | +/** | ||
| 6 | + * Simple back interface through which connection manager can interact with | ||
| 7 | + * the cluster store. | ||
| 8 | + */ | ||
| 9 | +public interface ClusterNodesDelegate { | ||
| 10 | + | ||
| 11 | + /** | ||
| 12 | + * Notifies about a new cluster node being detected. | ||
| 13 | + * | ||
| 14 | + * @param node newly detected cluster node | ||
| 15 | + */ | ||
| 16 | + void nodeDetected(DefaultControllerNode node); | ||
| 17 | + | ||
| 18 | + /** | ||
| 19 | + * Notifies about cluster node going offline. | ||
| 20 | + * | ||
| 21 | + * @param node cluster node that vanished | ||
| 22 | + */ | ||
| 23 | + void nodeVanished(DefaultControllerNode node); | ||
| 24 | + | ||
| 25 | +} |
core/store/dist/src/main/java/org/onlab/onos/store/cluster/impl/CommunicationsDelegate.java
0 → 100644
| 1 | +package org.onlab.onos.store.cluster.impl; | ||
| 2 | + | ||
| 3 | +import org.onlab.onos.store.cluster.messaging.ClusterMessage; | ||
| 4 | + | ||
| 5 | +/** | ||
| 6 | + * Simple back interface for interacting with the communications service. | ||
| 7 | + */ | ||
| 8 | +public interface CommunicationsDelegate { | ||
| 9 | + | ||
| 10 | + /** | ||
| 11 | + * Dispatches the specified message to all registered subscribers. | ||
| 12 | + * | ||
| 13 | + * @param message message to be dispatched | ||
| 14 | + */ | ||
| 15 | + void dispatch(ClusterMessage message); | ||
| 16 | + | ||
| 17 | + /** | ||
| 18 | + * Sets the sender. | ||
| 19 | + * | ||
| 20 | + * @param messageSender message sender | ||
| 21 | + */ | ||
| 22 | + void setSender(MessageSender messageSender); | ||
| 23 | + | ||
| 24 | +} |
| 1 | +package org.onlab.onos.store.cluster.impl; | ||
| 2 | + | ||
| 3 | +import org.onlab.onos.cluster.DefaultControllerNode; | ||
| 4 | +import org.onlab.onos.cluster.NodeId; | ||
| 5 | +import org.onlab.onos.store.cluster.messaging.ClusterMessage; | ||
| 6 | +import org.onlab.onos.store.cluster.messaging.ClusterMessageStream; | ||
| 7 | +import org.onlab.onos.store.cluster.messaging.HelloMessage; | ||
| 8 | +import org.onlab.onos.store.cluster.messaging.SerializationService; | ||
| 9 | +import org.slf4j.Logger; | ||
| 10 | +import org.slf4j.LoggerFactory; | ||
| 11 | + | ||
| 12 | +import java.io.IOException; | ||
| 13 | +import java.net.InetSocketAddress; | ||
| 14 | +import java.net.SocketAddress; | ||
| 15 | +import java.nio.channels.SocketChannel; | ||
| 16 | +import java.util.ArrayList; | ||
| 17 | +import java.util.HashSet; | ||
| 18 | +import java.util.List; | ||
| 19 | +import java.util.Map; | ||
| 20 | +import java.util.Set; | ||
| 21 | +import java.util.Timer; | ||
| 22 | +import java.util.TimerTask; | ||
| 23 | +import java.util.concurrent.ConcurrentHashMap; | ||
| 24 | +import java.util.concurrent.ExecutorService; | ||
| 25 | +import java.util.concurrent.Executors; | ||
| 26 | + | ||
| 27 | +import static java.net.InetAddress.getByAddress; | ||
| 28 | +import static org.onlab.util.Tools.namedThreads; | ||
| 29 | + | ||
| 30 | +/** | ||
| 31 | + * Manages connections to other controller cluster nodes. | ||
| 32 | + */ | ||
| 33 | +public class ConnectionManager implements MessageSender { | ||
| 34 | + | ||
| 35 | + private final Logger log = LoggerFactory.getLogger(getClass()); | ||
| 36 | + | ||
| 37 | + private static final long CONNECTION_CUSTODIAN_DELAY = 1000L; | ||
| 38 | + private static final long CONNECTION_CUSTODIAN_FREQUENCY = 5000; | ||
| 39 | + | ||
| 40 | + private static final long START_TIMEOUT = 1000; | ||
| 41 | + private static final int WORKERS = 3; | ||
| 42 | + | ||
| 43 | + private ClusterConnectionListener connectionListener; | ||
| 44 | + private List<ClusterIOWorker> workers = new ArrayList<>(WORKERS); | ||
| 45 | + | ||
| 46 | + private final DefaultControllerNode localNode; | ||
| 47 | + private final ClusterNodesDelegate nodesDelegate; | ||
| 48 | + private final CommunicationsDelegate commsDelegate; | ||
| 49 | + private final SerializationService serializationService; | ||
| 50 | + | ||
| 51 | + // Nodes to be monitored to make sure they have a connection. | ||
| 52 | + private final Set<DefaultControllerNode> nodes = new HashSet<>(); | ||
| 53 | + | ||
| 54 | + // Means to track message streams to other nodes. | ||
| 55 | + private final Map<NodeId, ClusterMessageStream> streams = new ConcurrentHashMap<>(); | ||
| 56 | + | ||
| 57 | + // Executor pools for listening and managing connections to other nodes. | ||
| 58 | + private final ExecutorService listenExecutor = | ||
| 59 | + Executors.newSingleThreadExecutor(namedThreads("onos-comm-listen")); | ||
| 60 | + private final ExecutorService commExecutors = | ||
| 61 | + Executors.newFixedThreadPool(WORKERS, namedThreads("onos-comm-cluster")); | ||
| 62 | + private final ExecutorService heartbeatExecutor = | ||
| 63 | + Executors.newSingleThreadExecutor(namedThreads("onos-comm-heartbeat")); | ||
| 64 | + | ||
| 65 | + private final Timer timer = new Timer("onos-comm-initiator"); | ||
| 66 | + private final TimerTask connectionCustodian = new ConnectionCustodian(); | ||
| 67 | + | ||
| 68 | + private final WorkerFinder workerFinder = new LeastUtilitiedWorkerFinder(); | ||
| 69 | + | ||
| 70 | + | ||
| 71 | + /** | ||
| 72 | + * Creates a new connection manager. | ||
| 73 | + */ | ||
| 74 | + ConnectionManager(DefaultControllerNode localNode, | ||
| 75 | + ClusterNodesDelegate nodesDelegate, | ||
| 76 | + CommunicationsDelegate commsDelegate, | ||
| 77 | + SerializationService serializationService) { | ||
| 78 | + this.localNode = localNode; | ||
| 79 | + this.nodesDelegate = nodesDelegate; | ||
| 80 | + this.commsDelegate = commsDelegate; | ||
| 81 | + this.serializationService = serializationService; | ||
| 82 | + | ||
| 83 | + commsDelegate.setSender(this); | ||
| 84 | + startCommunications(); | ||
| 85 | + startListening(); | ||
| 86 | + startInitiating(); | ||
| 87 | + log.info("Started"); | ||
| 88 | + } | ||
| 89 | + | ||
| 90 | + /** | ||
| 91 | + * Shuts down the connection manager. | ||
| 92 | + */ | ||
| 93 | + void shutdown() { | ||
| 94 | + connectionListener.shutdown(); | ||
| 95 | + for (ClusterIOWorker worker : workers) { | ||
| 96 | + worker.shutdown(); | ||
| 97 | + } | ||
| 98 | + log.info("Stopped"); | ||
| 99 | + } | ||
| 100 | + | ||
| 101 | + /** | ||
| 102 | + * Adds the node to the list of monitored nodes. | ||
| 103 | + * | ||
| 104 | + * @param node node to be added | ||
| 105 | + */ | ||
| 106 | + void addNode(DefaultControllerNode node) { | ||
| 107 | + nodes.add(node); | ||
| 108 | + } | ||
| 109 | + | ||
| 110 | + /** | ||
| 111 | + * Removes the node from the list of monitored nodes. | ||
| 112 | + * | ||
| 113 | + * @param node node to be removed | ||
| 114 | + */ | ||
| 115 | + void removeNode(DefaultControllerNode node) { | ||
| 116 | + nodes.remove(node); | ||
| 117 | + ClusterMessageStream stream = streams.remove(node.id()); | ||
| 118 | + if (stream != null) { | ||
| 119 | + stream.close(); | ||
| 120 | + } | ||
| 121 | + } | ||
| 122 | + | ||
| 123 | + /** | ||
| 124 | + * Removes the stream associated with the specified node. | ||
| 125 | + * | ||
| 126 | + * @param node node whose stream to remove | ||
| 127 | + */ | ||
| 128 | + void removeNodeStream(DefaultControllerNode node) { | ||
| 129 | + nodesDelegate.nodeVanished(node); | ||
| 130 | + streams.remove(node.id()); | ||
| 131 | + } | ||
| 132 | + | ||
| 133 | + @Override | ||
| 134 | + public boolean send(NodeId nodeId, ClusterMessage message) { | ||
| 135 | + ClusterMessageStream stream = streams.get(nodeId); | ||
| 136 | + if (stream != null) { | ||
| 137 | + try { | ||
| 138 | + stream.write(message); | ||
| 139 | + return true; | ||
| 140 | + } catch (IOException e) { | ||
| 141 | + log.warn("Unable to send a message about {} to node {}", | ||
| 142 | + message.subject(), nodeId); | ||
| 143 | + } | ||
| 144 | + } | ||
| 145 | + return false; | ||
| 146 | + } | ||
| 147 | + | ||
| 148 | + /** | ||
| 149 | + * Kicks off the IO loops and waits for them to startup. | ||
| 150 | + */ | ||
| 151 | + private void startCommunications() { | ||
| 152 | + HelloMessage hello = new HelloMessage(localNode.id(), localNode.ip(), | ||
| 153 | + localNode.tcpPort()); | ||
| 154 | + for (int i = 0; i < WORKERS; i++) { | ||
| 155 | + try { | ||
| 156 | + ClusterIOWorker worker = | ||
| 157 | + new ClusterIOWorker(this, commsDelegate, | ||
| 158 | + serializationService, hello); | ||
| 159 | + workers.add(worker); | ||
| 160 | + commExecutors.execute(worker); | ||
| 161 | + } catch (IOException e) { | ||
| 162 | + log.warn("Unable to start communication worker", e); | ||
| 163 | + } | ||
| 164 | + } | ||
| 165 | + | ||
| 166 | + // Wait for the IO loops to start | ||
| 167 | + for (ClusterIOWorker loop : workers) { | ||
| 168 | + if (!loop.awaitStart(START_TIMEOUT)) { | ||
| 169 | + log.warn("Comm loop did not start on-time; moving on..."); | ||
| 170 | + } | ||
| 171 | + } | ||
| 172 | + } | ||
| 173 | + | ||
| 174 | + /** | ||
| 175 | + * Starts listening for connections from peer cluster members. | ||
| 176 | + */ | ||
| 177 | + private void startListening() { | ||
| 178 | + try { | ||
| 179 | + connectionListener = | ||
| 180 | + new ClusterConnectionListener(localNode.ip(), localNode.tcpPort(), | ||
| 181 | + workerFinder); | ||
| 182 | + listenExecutor.execute(connectionListener); | ||
| 183 | + if (!connectionListener.awaitStart(START_TIMEOUT)) { | ||
| 184 | + log.warn("Listener did not start on-time; moving on..."); | ||
| 185 | + } | ||
| 186 | + } catch (IOException e) { | ||
| 187 | + log.error("Unable to listen for cluster connections", e); | ||
| 188 | + } | ||
| 189 | + } | ||
| 190 | + | ||
| 191 | + /** | ||
| 192 | + * Initiates open connection request and registers the pending socket | ||
| 193 | + * channel with the given IO loop. | ||
| 194 | + * | ||
| 195 | + * @param loop loop with which the channel should be registered | ||
| 196 | + * @throws java.io.IOException if the socket could not be open or connected | ||
| 197 | + */ | ||
| 198 | + private void initiateConnection(DefaultControllerNode node, | ||
| 199 | + ClusterIOWorker loop) throws IOException { | ||
| 200 | + SocketAddress sa = new InetSocketAddress(getByAddress(node.ip().toOctets()), node.tcpPort()); | ||
| 201 | + SocketChannel ch = SocketChannel.open(); | ||
| 202 | + ch.configureBlocking(false); | ||
| 203 | + ch.connect(sa); | ||
| 204 | + loop.connectStream(ch); | ||
| 205 | + } | ||
| 206 | + | ||
| 207 | + | ||
| 208 | + /** | ||
| 209 | + * Attempts to connect to any nodes that do not have an associated connection. | ||
| 210 | + */ | ||
| 211 | + private void startInitiating() { | ||
| 212 | + timer.schedule(connectionCustodian, CONNECTION_CUSTODIAN_DELAY, | ||
| 213 | + CONNECTION_CUSTODIAN_FREQUENCY); | ||
| 214 | + } | ||
| 215 | + | ||
| 216 | + // Sweeps through all controller nodes and attempts to open connection to | ||
| 217 | + // those that presently do not have one. | ||
| 218 | + private class ConnectionCustodian extends TimerTask { | ||
| 219 | + @Override | ||
| 220 | + public void run() { | ||
| 221 | + for (DefaultControllerNode node : nodes) { | ||
| 222 | + if (node != localNode && !streams.containsKey(node.id())) { | ||
| 223 | + try { | ||
| 224 | + initiateConnection(node, workerFinder.findWorker()); | ||
| 225 | + } catch (IOException e) { | ||
| 226 | + log.debug("Unable to connect", e); | ||
| 227 | + } | ||
| 228 | + } | ||
| 229 | + } | ||
| 230 | + } | ||
| 231 | + } | ||
| 232 | + | ||
| 233 | + // Finds the least utilitied IO loop. | ||
| 234 | + private class LeastUtilitiedWorkerFinder implements WorkerFinder { | ||
| 235 | + | ||
| 236 | + @Override | ||
| 237 | + public ClusterIOWorker findWorker() { | ||
| 238 | + ClusterIOWorker leastUtilized = null; | ||
| 239 | + int minCount = Integer.MAX_VALUE; | ||
| 240 | + for (ClusterIOWorker worker : workers) { | ||
| 241 | + int count = worker.streamCount(); | ||
| 242 | + if (count == 0) { | ||
| 243 | + return worker; | ||
| 244 | + } | ||
| 245 | + | ||
| 246 | + if (count < minCount) { | ||
| 247 | + leastUtilized = worker; | ||
| 248 | + minCount = count; | ||
| 249 | + } | ||
| 250 | + } | ||
| 251 | + return leastUtilized; | ||
| 252 | + } | ||
| 253 | + } | ||
| 254 | + | ||
| 255 | +} |
This diff is collapsed. Click to expand it.
| 1 | +package org.onlab.onos.store.cluster.impl; | ||
| 2 | + | ||
| 3 | +import org.onlab.onos.cluster.NodeId; | ||
| 4 | +import org.onlab.onos.store.cluster.messaging.ClusterMessage; | ||
| 5 | + | ||
| 6 | +/** | ||
| 7 | + * Created by tom on 9/29/14. | ||
| 8 | + */ | ||
| 9 | +public interface MessageSender { | ||
| 10 | + | ||
| 11 | + /** | ||
| 12 | + * Sends the specified message to the given cluster node. | ||
| 13 | + * | ||
| 14 | + * @param nodeId node identifier | ||
| 15 | + * @param message mesage to send | ||
| 16 | + * @return true if the message was sent sucessfully; false if there is | ||
| 17 | + * no stream or if there was an error | ||
| 18 | + */ | ||
| 19 | + boolean send(NodeId nodeId, ClusterMessage message); | ||
| 20 | + | ||
| 21 | +} |
| 1 | -package org.onlab.onos.store.cluster.impl; | ||
| 2 | - | ||
| 3 | -import org.onlab.nio.AbstractMessage; | ||
| 4 | - | ||
| 5 | -import java.util.Objects; | ||
| 6 | - | ||
| 7 | -import static com.google.common.base.MoreObjects.toStringHelper; | ||
| 8 | - | ||
| 9 | -/** | ||
| 10 | - * Base message for cluster-wide communications using TLVs. | ||
| 11 | - */ | ||
| 12 | -public class TLVMessage extends AbstractMessage { | ||
| 13 | - | ||
| 14 | - private final int type; | ||
| 15 | - private final byte[] data; | ||
| 16 | - | ||
| 17 | - /** | ||
| 18 | - * Creates an immutable TLV message. | ||
| 19 | - * | ||
| 20 | - * @param type message type | ||
| 21 | - * @param data message data bytes | ||
| 22 | - */ | ||
| 23 | - public TLVMessage(int type, byte[] data) { | ||
| 24 | - this.length = data.length + TLVMessageStream.METADATA_LENGTH; | ||
| 25 | - this.type = type; | ||
| 26 | - this.data = data; | ||
| 27 | - } | ||
| 28 | - | ||
| 29 | - /** | ||
| 30 | - * Returns the message type indicator. | ||
| 31 | - * | ||
| 32 | - * @return message type | ||
| 33 | - */ | ||
| 34 | - public int type() { | ||
| 35 | - return type; | ||
| 36 | - } | ||
| 37 | - | ||
| 38 | - /** | ||
| 39 | - * Returns the data bytes. | ||
| 40 | - * | ||
| 41 | - * @return message data | ||
| 42 | - */ | ||
| 43 | - public byte[] data() { | ||
| 44 | - return data; | ||
| 45 | - } | ||
| 46 | - | ||
| 47 | - @Override | ||
| 48 | - public int hashCode() { | ||
| 49 | - return Objects.hash(type, data); | ||
| 50 | - } | ||
| 51 | - | ||
| 52 | - @Override | ||
| 53 | - public boolean equals(Object obj) { | ||
| 54 | - if (this == obj) { | ||
| 55 | - return true; | ||
| 56 | - } | ||
| 57 | - if (obj == null || getClass() != obj.getClass()) { | ||
| 58 | - return false; | ||
| 59 | - } | ||
| 60 | - final TLVMessage other = (TLVMessage) obj; | ||
| 61 | - return Objects.equals(this.type, other.type) && | ||
| 62 | - Objects.equals(this.data, other.data); | ||
| 63 | - } | ||
| 64 | - | ||
| 65 | - @Override | ||
| 66 | - public String toString() { | ||
| 67 | - return toStringHelper(this).add("type", type).add("length", length).toString(); | ||
| 68 | - } | ||
| 69 | - | ||
| 70 | -} |
| 1 | +package org.onlab.onos.store.cluster.messaging; | ||
| 2 | + | ||
| 3 | +import org.onlab.onos.cluster.NodeId; | ||
| 4 | + | ||
| 5 | +import java.util.Set; | ||
| 6 | + | ||
| 7 | +/** | ||
| 8 | + * Service for assisting communications between controller cluster nodes. | ||
| 9 | + */ | ||
| 10 | +public interface ClusterCommunicationService { | ||
| 11 | + | ||
| 12 | + /** | ||
| 13 | + * Sends a message to the specified controller node. | ||
| 14 | + * | ||
| 15 | + * @param message message to send | ||
| 16 | + * @param toNodeId node identifier | ||
| 17 | + * @return true if the message was sent sucessfully; false if there is | ||
| 18 | + * no stream or if there was an error | ||
| 19 | + */ | ||
| 20 | + boolean send(ClusterMessage message, NodeId toNodeId); | ||
| 21 | + | ||
| 22 | + /** | ||
| 23 | + * Adds a new subscriber for the specified message subject. | ||
| 24 | + * | ||
| 25 | + * @param subject message subject | ||
| 26 | + * @param subscriber message subscriber | ||
| 27 | + */ | ||
| 28 | + void addSubscriber(MessageSubject subject, MessageSubscriber subscriber); | ||
| 29 | + | ||
| 30 | + /** | ||
| 31 | + * Removes the specified subscriber from the given message subject. | ||
| 32 | + * | ||
| 33 | + * @param subject message subject | ||
| 34 | + * @param subscriber message subscriber | ||
| 35 | + */ | ||
| 36 | + void removeSubscriber(MessageSubject subject, MessageSubscriber subscriber); | ||
| 37 | + | ||
| 38 | + /** | ||
| 39 | + * Returns the set of subscribers for the specified message subject. | ||
| 40 | + * | ||
| 41 | + * @param subject message subject | ||
| 42 | + * @return set of message subscribers | ||
| 43 | + */ | ||
| 44 | + Set<MessageSubscriber> getSubscribers(MessageSubject subject); | ||
| 45 | + | ||
| 46 | +} |
| 1 | +package org.onlab.onos.store.cluster.messaging; | ||
| 2 | + | ||
| 3 | +import org.onlab.nio.AbstractMessage; | ||
| 4 | + | ||
| 5 | +import static com.google.common.base.MoreObjects.toStringHelper; | ||
| 6 | + | ||
| 7 | +/** | ||
| 8 | + * Base message for cluster-wide communications. | ||
| 9 | + */ | ||
| 10 | +public abstract class ClusterMessage extends AbstractMessage { | ||
| 11 | + | ||
| 12 | + private final MessageSubject subject; | ||
| 13 | + | ||
| 14 | + /** | ||
| 15 | + * Creates a cluster message. | ||
| 16 | + * | ||
| 17 | + * @param subject message subject | ||
| 18 | + */ | ||
| 19 | + protected ClusterMessage(MessageSubject subject) { | ||
| 20 | + this.subject = subject; | ||
| 21 | + } | ||
| 22 | + | ||
| 23 | + /** | ||
| 24 | + * Returns the message subject indicator. | ||
| 25 | + * | ||
| 26 | + * @return message subject | ||
| 27 | + */ | ||
| 28 | + public MessageSubject subject() { | ||
| 29 | + return subject; | ||
| 30 | + } | ||
| 31 | + | ||
| 32 | + @Override | ||
| 33 | + public String toString() { | ||
| 34 | + return toStringHelper(this).add("subject", subject).add("length", length).toString(); | ||
| 35 | + } | ||
| 36 | + | ||
| 37 | +} |
| 1 | -package org.onlab.onos.store.cluster.impl; | 1 | +package org.onlab.onos.store.cluster.messaging; |
| 2 | 2 | ||
| 3 | import org.onlab.nio.IOLoop; | 3 | import org.onlab.nio.IOLoop; |
| 4 | import org.onlab.nio.MessageStream; | 4 | import org.onlab.nio.MessageStream; |
| ... | @@ -10,29 +10,29 @@ import java.nio.channels.ByteChannel; | ... | @@ -10,29 +10,29 @@ import java.nio.channels.ByteChannel; |
| 10 | import static com.google.common.base.Preconditions.checkState; | 10 | import static com.google.common.base.Preconditions.checkState; |
| 11 | 11 | ||
| 12 | /** | 12 | /** |
| 13 | - * Stream for transferring TLV messages between cluster members. | 13 | + * Stream for transferring messages between two cluster members. |
| 14 | */ | 14 | */ |
| 15 | -public class TLVMessageStream extends MessageStream<TLVMessage> { | 15 | +public class ClusterMessageStream extends MessageStream<ClusterMessage> { |
| 16 | 16 | ||
| 17 | - public static final int METADATA_LENGTH = 16; // 8 + 4 + 4 | 17 | + private static final int COMM_BUFFER_SIZE = 32 * 1024; |
| 18 | - | 18 | + private static final int COMM_IDLE_TIME = 500; |
| 19 | - private static final int LENGTH_OFFSET = 12; | ||
| 20 | - private static final long MARKER = 0xfeedcafecafefeedL; | ||
| 21 | 19 | ||
| 22 | private DefaultControllerNode node; | 20 | private DefaultControllerNode node; |
| 21 | + private SerializationService serializationService; | ||
| 23 | 22 | ||
| 24 | /** | 23 | /** |
| 25 | * Creates a message stream associated with the specified IO loop and | 24 | * Creates a message stream associated with the specified IO loop and |
| 26 | * backed by the given byte channel. | 25 | * backed by the given byte channel. |
| 27 | * | 26 | * |
| 28 | - * @param loop IO loop | 27 | + * @param serializationService service for encoding/decoding messages |
| 29 | - * @param byteChannel backing byte channel | 28 | + * @param loop IO loop |
| 30 | - * @param bufferSize size of the backing byte buffers | 29 | + * @param byteChannel backing byte channel |
| 31 | - * @param maxIdleMillis maximum number of millis the stream can be idle | ||
| 32 | */ | 30 | */ |
| 33 | - protected TLVMessageStream(IOLoop<TLVMessage, ?> loop, ByteChannel byteChannel, | 31 | + public ClusterMessageStream(SerializationService serializationService, |
| 34 | - int bufferSize, int maxIdleMillis) { | 32 | + IOLoop<ClusterMessage, ?> loop, |
| 35 | - super(loop, byteChannel, bufferSize, maxIdleMillis); | 33 | + ByteChannel byteChannel) { |
| 34 | + super(loop, byteChannel, COMM_BUFFER_SIZE, COMM_IDLE_TIME); | ||
| 35 | + this.serializationService = serializationService; | ||
| 36 | } | 36 | } |
| 37 | 37 | ||
| 38 | /** | 38 | /** |
| ... | @@ -40,7 +40,7 @@ public class TLVMessageStream extends MessageStream<TLVMessage> { | ... | @@ -40,7 +40,7 @@ public class TLVMessageStream extends MessageStream<TLVMessage> { |
| 40 | * | 40 | * |
| 41 | * @return controller node | 41 | * @return controller node |
| 42 | */ | 42 | */ |
| 43 | - DefaultControllerNode node() { | 43 | + public DefaultControllerNode node() { |
| 44 | return node; | 44 | return node; |
| 45 | } | 45 | } |
| 46 | 46 | ||
| ... | @@ -49,47 +49,19 @@ public class TLVMessageStream extends MessageStream<TLVMessage> { | ... | @@ -49,47 +49,19 @@ public class TLVMessageStream extends MessageStream<TLVMessage> { |
| 49 | * | 49 | * |
| 50 | * @param node controller node | 50 | * @param node controller node |
| 51 | */ | 51 | */ |
| 52 | - void setNode(DefaultControllerNode node) { | 52 | + public void setNode(DefaultControllerNode node) { |
| 53 | checkState(this.node == null, "Stream is already bound to a node"); | 53 | checkState(this.node == null, "Stream is already bound to a node"); |
| 54 | this.node = node; | 54 | this.node = node; |
| 55 | } | 55 | } |
| 56 | 56 | ||
| 57 | @Override | 57 | @Override |
| 58 | - protected TLVMessage read(ByteBuffer buffer) { | 58 | + protected ClusterMessage read(ByteBuffer buffer) { |
| 59 | - // Do we have enough bytes to read the header? If not, bail. | 59 | + return serializationService.decode(buffer); |
| 60 | - if (buffer.remaining() < METADATA_LENGTH) { | ||
| 61 | - return null; | ||
| 62 | - } | ||
| 63 | - | ||
| 64 | - // Peek at the length and if we have enough to read the entire message | ||
| 65 | - // go ahead, otherwise bail. | ||
| 66 | - int length = buffer.getInt(buffer.position() + LENGTH_OFFSET); | ||
| 67 | - if (buffer.remaining() < length) { | ||
| 68 | - return null; | ||
| 69 | - } | ||
| 70 | - | ||
| 71 | - // At this point, we have enough data to read a complete message. | ||
| 72 | - long marker = buffer.getLong(); | ||
| 73 | - checkState(marker == MARKER, "Incorrect message marker"); | ||
| 74 | - | ||
| 75 | - int type = buffer.getInt(); | ||
| 76 | - length = buffer.getInt(); | ||
| 77 | - | ||
| 78 | - // TODO: add deserialization hook here | ||
| 79 | - byte[] data = new byte[length - METADATA_LENGTH]; | ||
| 80 | - buffer.get(data); | ||
| 81 | - | ||
| 82 | - return new TLVMessage(type, data); | ||
| 83 | } | 60 | } |
| 84 | 61 | ||
| 85 | @Override | 62 | @Override |
| 86 | - protected void write(TLVMessage message, ByteBuffer buffer) { | 63 | + protected void write(ClusterMessage message, ByteBuffer buffer) { |
| 87 | - buffer.putLong(MARKER); | 64 | + serializationService.encode(message, buffer); |
| 88 | - buffer.putInt(message.type()); | ||
| 89 | - buffer.putInt(message.length()); | ||
| 90 | - | ||
| 91 | - // TODO: add serialization hook here | ||
| 92 | - buffer.put(message.data()); | ||
| 93 | } | 65 | } |
| 94 | 66 | ||
| 95 | } | 67 | } | ... | ... |
| 1 | +package org.onlab.onos.store.cluster.messaging; | ||
| 2 | + | ||
| 3 | +import org.onlab.onos.cluster.NodeId; | ||
| 4 | + | ||
| 5 | +/**l | ||
| 6 | + * Echo heart-beat message that nodes send to each other. | ||
| 7 | + */ | ||
| 8 | +public class EchoMessage extends ClusterMessage { | ||
| 9 | + | ||
| 10 | + private NodeId nodeId; | ||
| 11 | + | ||
| 12 | + // For serialization | ||
| 13 | + private EchoMessage() { | ||
| 14 | + super(MessageSubject.HELLO); | ||
| 15 | + nodeId = null; | ||
| 16 | + } | ||
| 17 | + | ||
| 18 | + /** | ||
| 19 | + * Creates a new heart-beat echo message. | ||
| 20 | + * | ||
| 21 | + * @param nodeId sending node identification | ||
| 22 | + */ | ||
| 23 | + public EchoMessage(NodeId nodeId) { | ||
| 24 | + super(MessageSubject.HELLO); | ||
| 25 | + nodeId = nodeId; | ||
| 26 | + } | ||
| 27 | + | ||
| 28 | + /** | ||
| 29 | + * Returns the sending node identifer. | ||
| 30 | + * | ||
| 31 | + * @return node identifier | ||
| 32 | + */ | ||
| 33 | + public NodeId nodeId() { | ||
| 34 | + return nodeId; | ||
| 35 | + } | ||
| 36 | + | ||
| 37 | +} |
| 1 | +package org.onlab.onos.store.cluster.messaging; | ||
| 2 | + | ||
| 3 | +import org.onlab.onos.cluster.NodeId; | ||
| 4 | +import org.onlab.packet.IpPrefix; | ||
| 5 | + | ||
| 6 | +/** | ||
| 7 | + * Hello message that nodes use to greet each other. | ||
| 8 | + */ | ||
| 9 | +public class HelloMessage extends ClusterMessage { | ||
| 10 | + | ||
| 11 | + private NodeId nodeId; | ||
| 12 | + private IpPrefix ipAddress; | ||
| 13 | + private int tcpPort; | ||
| 14 | + | ||
| 15 | + // For serialization | ||
| 16 | + private HelloMessage() { | ||
| 17 | + super(MessageSubject.HELLO); | ||
| 18 | + nodeId = null; | ||
| 19 | + ipAddress = null; | ||
| 20 | + tcpPort = 0; | ||
| 21 | + } | ||
| 22 | + | ||
| 23 | + /** | ||
| 24 | + * Creates a new hello message for the specified end-point data. | ||
| 25 | + * | ||
| 26 | + * @param nodeId sending node identification | ||
| 27 | + * @param ipAddress sending node IP address | ||
| 28 | + * @param tcpPort sending node TCP port | ||
| 29 | + */ | ||
| 30 | + public HelloMessage(NodeId nodeId, IpPrefix ipAddress, int tcpPort) { | ||
| 31 | + super(MessageSubject.HELLO); | ||
| 32 | + nodeId = nodeId; | ||
| 33 | + ipAddress = ipAddress; | ||
| 34 | + tcpPort = tcpPort; | ||
| 35 | + } | ||
| 36 | + | ||
| 37 | + /** | ||
| 38 | + * Returns the sending node identifer. | ||
| 39 | + * | ||
| 40 | + * @return node identifier | ||
| 41 | + */ | ||
| 42 | + public NodeId nodeId() { | ||
| 43 | + return nodeId; | ||
| 44 | + } | ||
| 45 | + | ||
| 46 | + /** | ||
| 47 | + * Returns the sending node IP address. | ||
| 48 | + * | ||
| 49 | + * @return node IP address | ||
| 50 | + */ | ||
| 51 | + public IpPrefix ipAddress() { | ||
| 52 | + return ipAddress; | ||
| 53 | + } | ||
| 54 | + | ||
| 55 | + /** | ||
| 56 | + * Returns the sending node TCP listen port. | ||
| 57 | + * | ||
| 58 | + * @return TCP listen port | ||
| 59 | + */ | ||
| 60 | + public int tcpPort() { | ||
| 61 | + return tcpPort; | ||
| 62 | + } | ||
| 63 | +} |
core/store/dist/src/main/java/org/onlab/onos/store/cluster/messaging/MessageSubscriber.java
0 → 100644
| 1 | +package org.onlab.onos.store.cluster.messaging; | ||
| 2 | + | ||
| 3 | +/** | ||
| 4 | + * Represents a message consumer. | ||
| 5 | + */ | ||
| 6 | +public interface MessageSubscriber { | ||
| 7 | + | ||
| 8 | + /** | ||
| 9 | + * Receives the specified cluster message. | ||
| 10 | + * | ||
| 11 | + * @param message message to be received | ||
| 12 | + */ | ||
| 13 | + void receive(ClusterMessage message); | ||
| 14 | + | ||
| 15 | +} |
core/store/dist/src/main/java/org/onlab/onos/store/cluster/messaging/SerializationService.java
0 → 100644
| 1 | +package org.onlab.onos.store.cluster.messaging; | ||
| 2 | + | ||
| 3 | +import java.nio.ByteBuffer; | ||
| 4 | + | ||
| 5 | +/** | ||
| 6 | + * Service for serializing/deserializing intra-cluster messages. | ||
| 7 | + */ | ||
| 8 | +public interface SerializationService { | ||
| 9 | + | ||
| 10 | + /** | ||
| 11 | + * Decodes the specified byte buffer to obtain a message within. | ||
| 12 | + * | ||
| 13 | + * @param buffer byte buffer with message(s) | ||
| 14 | + * @return parsed message | ||
| 15 | + */ | ||
| 16 | + ClusterMessage decode(ByteBuffer buffer); | ||
| 17 | + | ||
| 18 | + /** | ||
| 19 | + * Encodes the specified message into the given byte buffer. | ||
| 20 | + * | ||
| 21 | + * @param message message to be encoded | ||
| 22 | + * @param buffer byte buffer to receive the message data | ||
| 23 | + */ | ||
| 24 | + void encode(ClusterMessage message, ByteBuffer buffer); | ||
| 25 | + | ||
| 26 | +} |
| 1 | +package org.onlab.onos.store.cluster.messaging.impl; | ||
| 2 | + | ||
| 3 | +import com.google.common.collect.HashMultimap; | ||
| 4 | +import com.google.common.collect.ImmutableSet; | ||
| 5 | +import com.google.common.collect.Multimap; | ||
| 6 | +import org.apache.felix.scr.annotations.Component; | ||
| 7 | +import org.apache.felix.scr.annotations.Service; | ||
| 8 | +import org.onlab.onos.cluster.NodeId; | ||
| 9 | +import org.onlab.onos.store.cluster.impl.CommunicationsDelegate; | ||
| 10 | +import org.onlab.onos.store.cluster.impl.MessageSender; | ||
| 11 | +import org.onlab.onos.store.cluster.messaging.ClusterCommunicationService; | ||
| 12 | +import org.onlab.onos.store.cluster.messaging.ClusterMessage; | ||
| 13 | +import org.onlab.onos.store.cluster.messaging.MessageSubject; | ||
| 14 | +import org.onlab.onos.store.cluster.messaging.MessageSubscriber; | ||
| 15 | + | ||
| 16 | +import java.util.Set; | ||
| 17 | + | ||
| 18 | +/** | ||
| 19 | + * Implements the cluster communication services to use by other stores. | ||
| 20 | + */ | ||
| 21 | +@Component(immediate = true) | ||
| 22 | +@Service | ||
| 23 | +public class ClusterCommunicationManager | ||
| 24 | + implements ClusterCommunicationService, CommunicationsDelegate { | ||
| 25 | + | ||
| 26 | + // TODO: use something different that won't require synchronization | ||
| 27 | + private Multimap<MessageSubject, MessageSubscriber> subscribers = HashMultimap.create(); | ||
| 28 | + private MessageSender messageSender; | ||
| 29 | + | ||
| 30 | + @Override | ||
| 31 | + public boolean send(ClusterMessage message, NodeId toNodeId) { | ||
| 32 | + return messageSender.send(toNodeId, message); | ||
| 33 | + } | ||
| 34 | + | ||
| 35 | + @Override | ||
| 36 | + public synchronized void addSubscriber(MessageSubject subject, MessageSubscriber subscriber) { | ||
| 37 | + subscribers.put(subject, subscriber); | ||
| 38 | + } | ||
| 39 | + | ||
| 40 | + @Override | ||
| 41 | + public synchronized void removeSubscriber(MessageSubject subject, MessageSubscriber subscriber) { | ||
| 42 | + subscribers.remove(subject, subscriber); | ||
| 43 | + } | ||
| 44 | + | ||
| 45 | + @Override | ||
| 46 | + public Set<MessageSubscriber> getSubscribers(MessageSubject subject) { | ||
| 47 | + return ImmutableSet.copyOf(subscribers.get(subject)); | ||
| 48 | + } | ||
| 49 | + | ||
| 50 | + @Override | ||
| 51 | + public void dispatch(ClusterMessage message) { | ||
| 52 | + Set<MessageSubscriber> set = getSubscribers(message.subject()); | ||
| 53 | + if (set != null) { | ||
| 54 | + for (MessageSubscriber subscriber : set) { | ||
| 55 | + subscriber.receive(message); | ||
| 56 | + } | ||
| 57 | + } | ||
| 58 | + } | ||
| 59 | + | ||
| 60 | + @Override | ||
| 61 | + public void setSender(MessageSender messageSender) { | ||
| 62 | + this.messageSender = messageSender; | ||
| 63 | + } | ||
| 64 | +} |
core/store/dist/src/main/java/org/onlab/onos/store/cluster/messaging/impl/MessageSerializer.java
0 → 100644
| 1 | +package org.onlab.onos.store.cluster.messaging.impl; | ||
| 2 | + | ||
| 3 | +import org.onlab.onos.store.cluster.messaging.ClusterMessage; | ||
| 4 | +import org.onlab.onos.store.cluster.messaging.MessageSubject; | ||
| 5 | +import org.onlab.onos.store.cluster.messaging.SerializationService; | ||
| 6 | + | ||
| 7 | +import java.nio.ByteBuffer; | ||
| 8 | + | ||
| 9 | +import static com.google.common.base.Preconditions.checkState; | ||
| 10 | + | ||
| 11 | +/** | ||
| 12 | + * Factory for parsing messages sent between cluster members. | ||
| 13 | + */ | ||
| 14 | +public class MessageSerializer implements SerializationService { | ||
| 15 | + | ||
| 16 | + private static final int METADATA_LENGTH = 16; // 8 + 4 + 4 | ||
| 17 | + private static final int LENGTH_OFFSET = 12; | ||
| 18 | + | ||
| 19 | + private static final long MARKER = 0xfeedcafebeaddeadL; | ||
| 20 | + | ||
| 21 | + @Override | ||
| 22 | + public ClusterMessage decode(ByteBuffer buffer) { | ||
| 23 | + try { | ||
| 24 | + // Do we have enough bytes to read the header? If not, bail. | ||
| 25 | + if (buffer.remaining() < METADATA_LENGTH) { | ||
| 26 | + return null; | ||
| 27 | + } | ||
| 28 | + | ||
| 29 | + // Peek at the length and if we have enough to read the entire message | ||
| 30 | + // go ahead, otherwise bail. | ||
| 31 | + int length = buffer.getInt(buffer.position() + LENGTH_OFFSET); | ||
| 32 | + if (buffer.remaining() < length) { | ||
| 33 | + return null; | ||
| 34 | + } | ||
| 35 | + | ||
| 36 | + // At this point, we have enough data to read a complete message. | ||
| 37 | + long marker = buffer.getLong(); | ||
| 38 | + checkState(marker == MARKER, "Incorrect message marker"); | ||
| 39 | + | ||
| 40 | + int subjectOrdinal = buffer.getInt(); | ||
| 41 | + MessageSubject subject = MessageSubject.values()[subjectOrdinal]; | ||
| 42 | + length = buffer.getInt(); | ||
| 43 | + | ||
| 44 | + // TODO: sanity checking for length | ||
| 45 | + byte[] data = new byte[length - METADATA_LENGTH]; | ||
| 46 | + buffer.get(data); | ||
| 47 | + | ||
| 48 | + // TODO: add deserialization hook here; for now this hack | ||
| 49 | + return null; // actually deserialize | ||
| 50 | + | ||
| 51 | + } catch (Exception e) { | ||
| 52 | + // TODO: recover from exceptions by forwarding stream to next marker | ||
| 53 | + e.printStackTrace(); | ||
| 54 | + } | ||
| 55 | + return null; | ||
| 56 | + } | ||
| 57 | + | ||
| 58 | + @Override | ||
| 59 | + public void encode(ClusterMessage message, ByteBuffer buffer) { | ||
| 60 | + try { | ||
| 61 | + int i = 0; | ||
| 62 | + // Type based lookup for proper encoder | ||
| 63 | + } catch (Exception e) { | ||
| 64 | + // TODO: recover from exceptions by forwarding stream to next marker | ||
| 65 | + e.printStackTrace(); | ||
| 66 | + } | ||
| 67 | + } | ||
| 68 | + | ||
| 69 | +} |
-
Please register or login to post a comment