You are viewing a plain text version of this content. The canonical link for it is here.
Posted to notifications@asterixdb.apache.org by "Murtadha Hubail (Code Review)" <do...@asterixdb.incubator.apache.org> on 2016/05/30 23:42:18 UTC

Change in asterixdb[master]: Introduce MessagingNetworkManager for NC2NC AppMessaging

Murtadha Hubail has uploaded a new change for review.

  https://asterix-gerrit.ics.uci.edu/897

Change subject: Introduce MessagingNetworkManager for NC2NC AppMessaging
......................................................................

Introduce MessagingNetworkManager for NC2NC AppMessaging

This change introduces MessagingNetworkManager to NodeControllerService.
The MessagingNetworkManager is used to open channels that are used for NC2NC
application messaging. The messaging channels are passed to the application
through the IMessageBroker APIs and should be managed by the application itself.

Change-Id: I5c0bd7c11c1e78954ebceff49cb274d8073a64bd
---
M asterixdb/asterix-app/src/main/java/org/apache/asterix/api/common/AsterixHyracksIntegrationUtil.java
M asterixdb/asterix-app/src/main/java/org/apache/asterix/messaging/NCMessageBroker.java
A asterixdb/asterix-app/src/main/java/org/apache/asterix/messaging/NCMessagingHelper.java
M asterixdb/asterix-common/src/main/java/org/apache/asterix/common/messaging/api/IApplicationMessage.java
M asterixdb/asterix-common/src/main/java/org/apache/asterix/common/messaging/api/INCMessageBroker.java
M asterixdb/asterix-transactions/src/main/java/org/apache/asterix/transaction/management/resource/GlobalResourceIdFactory.java
M hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/client/NodeControllerInfo.java
R hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/IBufferAcceptor.java
R hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/IBufferFactory.java
C hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/IChannelControlBlock.java
R hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/IChannelReadInterface.java
R hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/IChannelWriteInterface.java
R hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/ICloseableBufferAcceptor.java
M hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/messages/IMessageBroker.java
M hyracks-fullstack/hyracks/hyracks-comm/src/main/java/org/apache/hyracks/comm/channels/DatasetNetworkInputChannel.java
A hyracks-fullstack/hyracks/hyracks-comm/src/main/java/org/apache/hyracks/comm/channels/NcMessagingUtil.java
M hyracks-fullstack/hyracks/hyracks-comm/src/main/java/org/apache/hyracks/comm/channels/NetworkInputChannel.java
M hyracks-fullstack/hyracks/hyracks-comm/src/main/java/org/apache/hyracks/comm/channels/NetworkOutputChannel.java
M hyracks-fullstack/hyracks/hyracks-comm/src/main/java/org/apache/hyracks/comm/channels/ReadBufferFactory.java
M hyracks-fullstack/hyracks/hyracks-control/hyracks-control-cc/src/main/java/org/apache/hyracks/control/cc/NodeControllerState.java
M hyracks-fullstack/hyracks/hyracks-control/hyracks-control-cc/src/main/java/org/apache/hyracks/control/cc/work/GetNodeControllersInfoWork.java
M hyracks-fullstack/hyracks/hyracks-control/hyracks-control-common/src/main/java/org/apache/hyracks/control/common/controllers/NCConfig.java
M hyracks-fullstack/hyracks/hyracks-control/hyracks-control-common/src/main/java/org/apache/hyracks/control/common/controllers/NodeRegistration.java
M hyracks-fullstack/hyracks/hyracks-control/hyracks-control-nc/src/main/java/org/apache/hyracks/control/nc/NodeControllerService.java
M hyracks-fullstack/hyracks/hyracks-control/hyracks-control-nc/src/main/java/org/apache/hyracks/control/nc/net/DatasetNetworkManager.java
A hyracks-fullstack/hyracks/hyracks-control/hyracks-control-nc/src/main/java/org/apache/hyracks/control/nc/net/MessagingNetworkManager.java
M hyracks-fullstack/hyracks/hyracks-control/hyracks-control-nc/src/main/java/org/apache/hyracks/control/nc/net/NetworkManager.java
M hyracks-fullstack/hyracks/hyracks-hdfs/hyracks-hdfs-core/src/test/java/org/apache/hyracks/hdfs/scheduler/SchedulerTest.java
M hyracks-fullstack/hyracks/hyracks-hdfs/hyracks-hdfs-core/src/test/java/org/apache/hyracks/hdfs/utils/TestUtils.java
M hyracks-fullstack/hyracks/hyracks-hdfs/hyracks-hdfs-core/src/test/java/org/apache/hyracks/hdfs2/scheduler/SchedulerTest.java
M hyracks-fullstack/hyracks/hyracks-net/pom.xml
M hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/protocols/muxdemux/ChannelControlBlock.java
M hyracks-fullstack/hyracks/hyracks-net/src/test/java/org/apache/hyracks/net/tests/NetTest.java
33 files changed, 848 insertions(+), 200 deletions(-)


  git pull ssh://asterix-gerrit.ics.uci.edu:29418/asterixdb refs/changes/97/897/1

diff --git a/asterixdb/asterix-app/src/main/java/org/apache/asterix/api/common/AsterixHyracksIntegrationUtil.java b/asterixdb/asterix-app/src/main/java/org/apache/asterix/api/common/AsterixHyracksIntegrationUtil.java
index cc50b75..0cc9570 100644
--- a/asterixdb/asterix-app/src/main/java/org/apache/asterix/api/common/AsterixHyracksIntegrationUtil.java
+++ b/asterixdb/asterix-app/src/main/java/org/apache/asterix/api/common/AsterixHyracksIntegrationUtil.java
@@ -82,6 +82,7 @@
             ncConfig1.clusterNetIPAddress = "127.0.0.1";
             ncConfig1.dataIPAddress = "127.0.0.1";
             ncConfig1.resultIPAddress = "127.0.0.1";
+            ncConfig1.messagingIPAddress = "127.0.0.1";
             ncConfig1.nodeId = ncName;
             ncConfig1.resultTTL = 30000;
             ncConfig1.resultSweepThreshold = 1000;
diff --git a/asterixdb/asterix-app/src/main/java/org/apache/asterix/messaging/NCMessageBroker.java b/asterixdb/asterix-app/src/main/java/org/apache/asterix/messaging/NCMessageBroker.java
index 13b0189..836946a 100644
--- a/asterixdb/asterix-app/src/main/java/org/apache/asterix/messaging/NCMessageBroker.java
+++ b/asterixdb/asterix-app/src/main/java/org/apache/asterix/messaging/NCMessageBroker.java
@@ -18,8 +18,10 @@
  */
 package org.apache.asterix.messaging;
 
+import java.util.HashMap;
 import java.util.Map;
 import java.util.concurrent.ConcurrentHashMap;
+import java.util.concurrent.LinkedBlockingQueue;
 import java.util.concurrent.atomic.AtomicLong;
 import java.util.logging.Level;
 import java.util.logging.Logger;
@@ -44,40 +46,66 @@
 import org.apache.asterix.event.schema.cluster.Node;
 import org.apache.asterix.metadata.bootstrap.MetadataIndexImmutableProperties;
 import org.apache.asterix.transaction.management.resource.PersistentLocalResourceRepository;
+import org.apache.hyracks.api.comm.IChannelControlBlock;
 import org.apache.hyracks.api.messages.IMessage;
 import org.apache.hyracks.api.util.JavaSerializationUtils;
 import org.apache.hyracks.control.nc.NodeControllerService;
 
 public class NCMessageBroker implements INCMessageBroker {
-    private final static Logger LOGGER = Logger.getLogger(NCMessageBroker.class.getName());
+    private static final Logger LOGGER = Logger.getLogger(NCMessageBroker.class.getName());
 
     private final NodeControllerService ncs;
     private final AtomicLong messageId = new AtomicLong(0);
     private final Map<Long, IApplicationMessageCallback> callbacks;
     private final IAsterixAppRuntimeContext appContext;
+    private final Map<String, IChannelControlBlock> ncChannels = new HashMap<>();
+    private final NCMessagingHelper messagingHelper;
+    private final LinkedBlockingQueue<IApplicationMessage> receivedMsgsQ;
+    private final MessageDeliveryService msgDeliverySvc;
 
     public NCMessageBroker(NodeControllerService ncs) {
         this.ncs = ncs;
         appContext = (IAsterixAppRuntimeContext) ncs.getApplicationContext().getApplicationObject();
-        callbacks = new ConcurrentHashMap<Long, IApplicationMessageCallback>();
+        callbacks = new ConcurrentHashMap<>();
+        messagingHelper = new NCMessagingHelper(ncs, this);
+        receivedMsgsQ = new LinkedBlockingQueue<>();
+        msgDeliverySvc = new MessageDeliveryService();
+        appContext.getThreadExecutor().execute(msgDeliverySvc);
     }
 
     @Override
-    public void sendMessage(IApplicationMessage message, IApplicationMessageCallback callback) throws Exception {
-        if (callback != null) {
-            long uniqueMessageId = messageId.incrementAndGet();
-            message.setId(uniqueMessageId);
-            callbacks.put(uniqueMessageId, callback);
-        }
+    public void sendMessageToCC(IApplicationMessage message, IApplicationMessageCallback callback) throws Exception {
+        registerMsgCallback(message, callback);
         try {
             ncs.sendApplicationMessageToCC(JavaSerializationUtils.serialize(message), null);
         } catch (Exception e) {
-            if (callback != null) {
-                //remove the callback in case of failure
-                callbacks.remove(message.getId());
-            }
+            handleMsgDeliveryFailure(message);
             throw e;
         }
+    }
+
+    @Override
+    public void sendMessageToNC(String nodeId, IApplicationMessage message, IApplicationMessageCallback callback)
+            throws Exception {
+        registerMsgCallback(message, callback);
+        try {
+            IChannelControlBlock ccb = getNCChannel(nodeId);
+            messagingHelper.sendMessageToChannel(ccb, message);
+        } catch (Exception e) {
+            handleMsgDeliveryFailure(message);
+            throw e;
+        }
+    }
+
+    @Override
+    public void registerMessagingChannel(String nodeId, IChannelControlBlock ccb) {
+        messagingHelper.configureChannel(ccb);
+        addOpenChannel(nodeId, ccb);
+    }
+
+    @Override
+    public void queueReceivedMessage(IApplicationMessage msg) {
+        receivedMsgsQ.offer(msg);
     }
 
     @Override
@@ -117,8 +145,54 @@
                     break;
             }
         } catch (Exception e) {
-            e.printStackTrace();
+            if (LOGGER.isLoggable(Level.WARNING)) {
+                LOGGER.log(Level.WARNING, e.getMessage(), e);
+            }
             throw e;
+        }
+    }
+
+    private void registerMsgCallback(IApplicationMessage message, IApplicationMessageCallback callback) {
+        if (callback != null) {
+            long uniqueMessageId = messageId.incrementAndGet();
+            message.setId(uniqueMessageId);
+            callbacks.put(uniqueMessageId, callback);
+        }
+    }
+
+    private void handleMsgDeliveryFailure(IApplicationMessage message) {
+        callbacks.remove(message.getId());
+    }
+
+    private IChannelControlBlock getNCChannel(String nodeId) throws Exception {
+        synchronized (ncChannels) {
+            // check if there is an existing open channel first
+            IChannelControlBlock ncChannel = ncChannels.get(nodeId);
+            if (ncChannels.get(nodeId) == null) {
+                //open a new channel and register it
+                ncChannel = messagingHelper.openChannel(nodeId);
+                registerMessagingChannel(nodeId, ncChannel);
+            }
+            return ncChannel;
+        }
+    }
+
+    private void addOpenChannel(String nodeId, IChannelControlBlock ccb) {
+        synchronized (ncChannels) {
+            if (ncChannels.get(nodeId) == null) {
+                ncChannels.put(nodeId, ccb);
+            } else {
+                //close this channel since another channel was opened with this node
+                messagingHelper.closeChannel(ccb);
+                /**
+                 * TODO Currently there is a chance that two nodes will open
+                 * a channel to each other at the exact same time and both will
+                 * end up using a half closed channel. While this isn't a big issue,
+                 * it should be eliminated by introducing negotiation protocol
+                 * between nodes to decide which channel to use and which channel
+                 * to close fully.
+                 */
+            }
         }
     }
 
@@ -133,7 +207,7 @@
                 //send response after takeover is completed
                 TakeoverPartitionsResponseMessage reponse = new TakeoverPartitionsResponseMessage(msg.getRequestId(),
                         appContext.getTransactionSubsystem().getId(), msg.getPartitions());
-                sendMessage(reponse, null);
+                sendMessageToCC(reponse, null);
             }
         }
     }
@@ -145,7 +219,7 @@
         } finally {
             TakeoverMetadataNodeResponseMessage reponse = new TakeoverMetadataNodeResponseMessage(
                     appContext.getTransactionSubsystem().getId());
-            sendMessage(reponse, null);
+            sendMessageToCC(reponse, null);
         }
     }
 
@@ -156,7 +230,7 @@
         long maxResourceId = Math.max(appContext.getLocalResourceRepository().getMaxResourceID(),
                 MetadataIndexImmutableProperties.FIRST_AVAILABLE_USER_DATASET_ID);
         maxResourceIdMsg.setMaxResourceId(maxResourceId);
-        sendMessage(maxResourceIdMsg, null);
+        sendMessageToCC(maxResourceIdMsg, null);
     }
 
     private void handleReplicaEvent(IMessage message) {
@@ -198,7 +272,7 @@
         //send response after partitions prepared for failback
         PreparePartitionsFailbackResponseMessage reponse = new PreparePartitionsFailbackResponseMessage(msg.getPlanId(),
                 msg.getRequestId(), msg.getPartitions());
-        sendMessage(reponse, null);
+        sendMessageToCC(reponse, null);
     }
 
     private void handleCompleteFailbackRequest(IMessage message) throws Exception {
@@ -209,7 +283,26 @@
         } finally {
             CompleteFailbackResponseMessage reponse = new CompleteFailbackResponseMessage(msg.getPlanId(),
                     msg.getRequestId(), msg.getPartitions());
-            sendMessage(reponse, null);
+            sendMessageToCC(reponse, null);
+        }
+    }
+
+    private class MessageDeliveryService implements Runnable {
+        @Override
+        public void run() {
+            while (true) {
+                try {
+                    IApplicationMessage msg = receivedMsgsQ.take();
+                    //TODO add nodeId to IApplicationMessage and pass it
+                    receivedMessage(msg, null);
+                } catch (InterruptedException e) {
+                    Thread.currentThread().interrupt();
+                } catch (Exception e) {
+                    if (LOGGER.isLoggable(Level.WARNING)) {
+                        LOGGER.log(Level.WARNING, "Could not process message", e);
+                    }
+                }
+            }
         }
     }
 }
diff --git a/asterixdb/asterix-app/src/main/java/org/apache/asterix/messaging/NCMessagingHelper.java b/asterixdb/asterix-app/src/main/java/org/apache/asterix/messaging/NCMessagingHelper.java
new file mode 100644
index 0000000..a8908aa
--- /dev/null
+++ b/asterixdb/asterix-app/src/main/java/org/apache/asterix/messaging/NCMessagingHelper.java
@@ -0,0 +1,217 @@
+package org.apache.asterix.messaging;
+
+import java.io.IOException;
+import java.net.InetAddress;
+import java.net.InetSocketAddress;
+import java.net.SocketAddress;
+import java.nio.ByteBuffer;
+import java.util.Map;
+import java.util.logging.Level;
+import java.util.logging.Logger;
+
+import org.apache.asterix.common.messaging.api.IApplicationMessage;
+import org.apache.asterix.common.messaging.api.INCMessageBroker;
+import org.apache.asterix.external.feed.dataflow.FrameAction;
+import org.apache.asterix.external.feed.management.ConcurrentFramePool;
+import org.apache.hyracks.api.client.NodeControllerInfo;
+import org.apache.hyracks.api.comm.IBufferAcceptor;
+import org.apache.hyracks.api.comm.IBufferFactory;
+import org.apache.hyracks.api.comm.IChannelControlBlock;
+import org.apache.hyracks.api.comm.ICloseableBufferAcceptor;
+import org.apache.hyracks.api.comm.NetworkAddress;
+import org.apache.hyracks.api.exceptions.HyracksDataException;
+import org.apache.hyracks.api.util.JavaSerializationUtils;
+import org.apache.hyracks.control.nc.NodeControllerService;
+import org.apache.hyracks.net.protocols.muxdemux.ChannelControlBlock;
+import org.apache.hyracks.util.StorageUtil;
+import org.apache.hyracks.util.StorageUtil.StorageUnit;
+
+public class NCMessagingHelper {
+
+    private static final Logger LOGGER = Logger.getLogger(NCMessagingHelper.class.getName());
+    //TODO make these values configurable and account for their memory usage
+    private static final int MESSAGING_BUFFER_SIZE = StorageUtil.getSizeInBytes(4, StorageUnit.KILOBYTE);
+    private static final int MESSAGING_MEMORY_BUDGET = 512 * MESSAGING_BUFFER_SIZE;
+    private static final int MAX_MESSAGING_BUFFER_PER_CHANNEL = 16;
+
+    /**
+     * A buffer type to indicate whether an empty buffer was used during
+     * a write or read operation.
+     */
+    private enum AppMessagingEmptyBufferType {
+        READ_BUFFER,
+        WRITE_BUFFER
+    }
+
+    /* A single buffer factory used by all messaging channels */
+    private final IBufferFactory appMessagingBufferFactor = new AppMessagingBufferFactory();
+
+    private final INCMessageBroker messageBroker;
+    private final ConcurrentFramePool framePool;
+    private final NodeControllerService ncs;
+
+    public NCMessagingHelper(NodeControllerService ncs, INCMessageBroker messageBroker) {
+        this.ncs = ncs;
+        this.messageBroker = messageBroker;
+        framePool = new ConcurrentFramePool(ncs.getId(), MESSAGING_MEMORY_BUDGET, MESSAGING_BUFFER_SIZE);
+    }
+
+    public IChannelControlBlock openChannel(String nodeId) throws Exception {
+        // Get active nodes info from CC
+        //TODO Don't send a request to CC for node info each time but only on node membership changes.
+        Map<String, NodeControllerInfo> nodeControllers = ncs.getNodeControllersInfo();
+
+        // Get the node messaging address from its info
+        NodeControllerInfo nodeControllerInfo = nodeControllers.get(nodeId);
+        if (nodeControllerInfo == null) {
+            throw new HyracksDataException("Could not find node: " + nodeId);
+        }
+        NetworkAddress nodeMessagingNeAddress = nodeControllerInfo.getMessagingNetworkAddress();
+        SocketAddress nodeAddress = new InetSocketAddress(InetAddress.getByName(nodeMessagingNeAddress.getAddress()),
+                nodeMessagingNeAddress.getPort());
+
+        // Open a channel using MessagingNetManager 
+        IChannelControlBlock ccb = ncs.getMessagingNetManager().connect(nodeAddress);
+        //Send the initial app messaging handshake message to register the opened channel on both nodes
+        ByteBuffer initialBuffer = ncs.getMessagingNetManager().constructAppMessagingInitialMessage(ncs.getId());
+        ccb.getWriteInterface().getFullBufferAcceptor().accept(initialBuffer);
+        return ccb;
+    }
+
+    public void configureChannel(IChannelControlBlock ccb) {
+        /**
+         * Set the app messaging buffer factory and acceptors
+         * to the channel read and write interfaces
+         */
+        ccb.getWriteInterface().setEmptyBufferAcceptor(
+                new AppMessagingEmptyBufferAcceptor(ccb, AppMessagingEmptyBufferType.WRITE_BUFFER));
+        ccb.getReadInterface().setBufferFactory(appMessagingBufferFactor, MAX_MESSAGING_BUFFER_PER_CHANNEL,
+                MESSAGING_BUFFER_SIZE);
+        AppMessagingEmptyBufferAcceptor readEmptyBufferAcceptor = new AppMessagingEmptyBufferAcceptor(ccb,
+                AppMessagingEmptyBufferType.READ_BUFFER);
+        ccb.getReadInterface().setEmptyBufferAcceptor(readEmptyBufferAcceptor);
+        ccb.getReadInterface().setFullBufferAcceptor(new AppMessagingReadFullBufferAcceptor(readEmptyBufferAcceptor));
+    }
+
+    public void sendMessageToChannel(IChannelControlBlock ccb, IApplicationMessage msg) throws IOException {
+        byte[] serializedMsg = JavaSerializationUtils.serialize(msg);
+        if (serializedMsg.length > MESSAGING_BUFFER_SIZE) {
+            throw new HyracksDataException("Message exceded maximum size");
+        }
+        // Prepare the message buffer
+        ByteBuffer msgBuffer = framePool.get();
+        if (msgBuffer == null) {
+            throw new HyracksDataException("Could not get an empty buffer");
+        }
+        msgBuffer.clear();
+        msgBuffer.put(serializedMsg);
+        msgBuffer.flip();
+        // Give the buffer to the channel write interface for writing
+        ccb.getWriteInterface().getFullBufferAcceptor().accept(msgBuffer);
+    }
+
+    public void closeChannel(IChannelControlBlock ccb) {
+        /**
+         * Close the channel write interface which will result in
+         * sending a CLOSE_CHANNEL request to the remote node of
+         * the channel.
+         */
+        ccb.getWriteInterface().getFullBufferAcceptor().close();
+    }
+
+    /**
+     * A buffer acceptor that receives the read buffers containing messages from
+     * other nodes.
+     */
+    private class AppMessagingReadFullBufferAcceptor implements ICloseableBufferAcceptor {
+        private final IBufferAcceptor recycle;
+
+        private AppMessagingReadFullBufferAcceptor(IBufferAcceptor recycle) {
+            this.recycle = recycle;
+        }
+
+        @Override
+        public void accept(ByteBuffer buffer) {
+            try {
+                IApplicationMessage receivedMsg = (IApplicationMessage) JavaSerializationUtils
+                        .deserialize(buffer.array());
+                //queue the received message and free the network IO thread
+                messageBroker.queueReceivedMessage(receivedMsg);
+            } catch (ClassNotFoundException | IOException e) {
+                if (LOGGER.isLoggable(Level.WARNING)) {
+                    LOGGER.log(Level.WARNING, e.getMessage(), e);
+                }
+            } finally {
+                recycle.accept(buffer);
+            }
+        }
+
+        @Override
+        public void close() {
+        }
+
+        @Override
+        public void error(int ecode) {
+        }
+    }
+
+    /**
+     * Empty buffer acceptor used to return the used buffers in app messaging
+     * to the buffer pool.
+     */
+    private class AppMessagingEmptyBufferAcceptor implements IBufferAcceptor {
+        private final AppMessagingEmptyBufferType bufferType;
+        private final ChannelControlBlock ccb;
+
+        public AppMessagingEmptyBufferAcceptor(IChannelControlBlock ccb, AppMessagingEmptyBufferType bufferType) {
+            this.ccb = (ChannelControlBlock) ccb;
+            this.bufferType = bufferType;
+        }
+
+        @Override
+        public void accept(ByteBuffer buffer) {
+            try {
+                int remaining = buffer.remaining();
+                if (bufferType == AppMessagingEmptyBufferType.READ_BUFFER) {
+                    /**
+                     * add the number of received bytes from this channel
+                     * as write credits to make sure the messaging channel
+                     * full write credit is always available.
+                     */
+                    ccb.addWriteCredits(remaining);
+                }
+                framePool.release(buffer);
+            } catch (HyracksDataException e) {
+                if (LOGGER.isLoggable(Level.WARNING)) {
+                    LOGGER.log(Level.WARNING, e.getMessage(), e);
+                }
+            }
+        }
+    }
+
+    /**
+     * A buffer factory based on {@link ConcurrentFramePool}. Used
+     * for messaging channels buffers.
+     */
+    private final class AppMessagingBufferFactory implements IBufferFactory {
+        private final FrameAction frameAction = new FrameAction();
+
+        @Override
+        public ByteBuffer createBuffer() {
+            ByteBuffer buffer = framePool.get();
+            if (buffer == null) {
+                try {
+                    framePool.subscribe(frameAction);
+                    buffer = frameAction.retrieve();
+                } catch (HyracksDataException e) {
+                    if (LOGGER.isLoggable(Level.WARNING)) {
+                        LOGGER.log(Level.WARNING, e.getMessage(), e);
+                    }
+                } catch (InterruptedException e) {
+                    Thread.currentThread().interrupt();
+                }
+            }
+            return buffer;
+        }
+    }
+}
\ No newline at end of file
diff --git a/asterixdb/asterix-common/src/main/java/org/apache/asterix/common/messaging/api/IApplicationMessage.java b/asterixdb/asterix-common/src/main/java/org/apache/asterix/common/messaging/api/IApplicationMessage.java
index fba74e8..bcf7a8c 100644
--- a/asterixdb/asterix-common/src/main/java/org/apache/asterix/common/messaging/api/IApplicationMessage.java
+++ b/asterixdb/asterix-common/src/main/java/org/apache/asterix/common/messaging/api/IApplicationMessage.java
@@ -43,7 +43,7 @@
 
     /**
      * Sets a unique message id that identifies this message within an NC.
-     * This id is set by {@link INCMessageBroker#sendMessage(IApplicationMessage, IApplicationMessageCallback)}
+     * This id is set by {@link INCMessageBroker#sendMessageToCC(IApplicationMessage, IApplicationMessageCallback)}
      * when the callback is not null to notify the sender when the response to that message is received.
      *
      * @param messageId
diff --git a/asterixdb/asterix-common/src/main/java/org/apache/asterix/common/messaging/api/INCMessageBroker.java b/asterixdb/asterix-common/src/main/java/org/apache/asterix/common/messaging/api/INCMessageBroker.java
index 41f8a0c..7628167 100644
--- a/asterixdb/asterix-common/src/main/java/org/apache/asterix/common/messaging/api/INCMessageBroker.java
+++ b/asterixdb/asterix-common/src/main/java/org/apache/asterix/common/messaging/api/INCMessageBroker.java
@@ -29,7 +29,17 @@
      * @param callback
      * @throws Exception
      */
-    public void sendMessage(IApplicationMessage message, IApplicationMessageCallback callback) throws Exception;
+    public void sendMessageToCC(IApplicationMessage message, IApplicationMessageCallback callback) throws Exception;
+
+    /**
+     * Sends application message from this NC to another NC.
+     *
+     * @param message
+     * @param callback
+     * @throws Exception
+     */
+    public void sendMessageToNC(String nodeId, IApplicationMessage message, IApplicationMessageCallback callback)
+            throws Exception;
 
     /**
      * Sends the maximum resource id on this NC to the CC.
@@ -37,4 +47,11 @@
      * @throws Exception
      */
     public void reportMaxResourceId() throws Exception;
+
+    /**
+     * Queue a message to this {@link INCMessageBroker} for processing
+     *
+     * @param msg
+     */
+    public void queueReceivedMessage(IApplicationMessage msg);
 }
diff --git a/asterixdb/asterix-transactions/src/main/java/org/apache/asterix/transaction/management/resource/GlobalResourceIdFactory.java b/asterixdb/asterix-transactions/src/main/java/org/apache/asterix/transaction/management/resource/GlobalResourceIdFactory.java
index 5b29530..ab1ebe1 100644
--- a/asterixdb/asterix-transactions/src/main/java/org/apache/asterix/transaction/management/resource/GlobalResourceIdFactory.java
+++ b/asterixdb/asterix-transactions/src/main/java/org/apache/asterix/transaction/management/resource/GlobalResourceIdFactory.java
@@ -57,7 +57,7 @@
             //if no response available or it has an exception, request a new one
             if (reponse == null || reponse.getException() != null) {
                 ResourceIdRequestMessage msg = new ResourceIdRequestMessage();
-                ((INCMessageBroker) appCtx.getMessageBroker()).sendMessage(msg, this);
+                ((INCMessageBroker) appCtx.getMessageBroker()).sendMessageToCC(msg, this);
                 reponse = (ResourceIdRequestResponseMessage) resourceIdResponseQ.take();
                 if (reponse.getException() != null) {
                     throw new HyracksDataException(reponse.getException().getMessage());
diff --git a/hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/client/NodeControllerInfo.java b/hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/client/NodeControllerInfo.java
index c41dafe..a79b955 100644
--- a/hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/client/NodeControllerInfo.java
+++ b/hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/client/NodeControllerInfo.java
@@ -33,12 +33,15 @@
 
     private final NetworkAddress datasetNetworkAddress;
 
+    private final NetworkAddress messagingNetworkAddress;
+
     public NodeControllerInfo(String nodeId, NodeStatus status, NetworkAddress netAddress,
-            NetworkAddress datasetNetworkAddress) {
+            NetworkAddress datasetNetworkAddress, NetworkAddress messagingNetworkAddress) {
         this.nodeId = nodeId;
         this.status = status;
         this.netAddress = netAddress;
         this.datasetNetworkAddress = datasetNetworkAddress;
+        this.messagingNetworkAddress = messagingNetworkAddress;
     }
 
     public String getNodeId() {
@@ -56,4 +59,8 @@
     public NetworkAddress getDatasetNetworkAddress() {
         return datasetNetworkAddress;
     }
+
+    public NetworkAddress getMessagingNetworkAddress() {
+        return messagingNetworkAddress;
+    }
 }
diff --git a/hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/buffers/IBufferAcceptor.java b/hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/IBufferAcceptor.java
similarity index 95%
rename from hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/buffers/IBufferAcceptor.java
rename to hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/IBufferAcceptor.java
index d7febd2..d69ddd7 100644
--- a/hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/buffers/IBufferAcceptor.java
+++ b/hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/IBufferAcceptor.java
@@ -16,7 +16,7 @@
  * specific language governing permissions and limitations
  * under the License.
  */
-package org.apache.hyracks.net.buffers;
+package org.apache.hyracks.api.comm;
 
 import java.nio.ByteBuffer;
 
diff --git a/hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/protocols/muxdemux/IBufferFactory.java b/hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/IBufferFactory.java
similarity index 94%
rename from hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/protocols/muxdemux/IBufferFactory.java
rename to hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/IBufferFactory.java
index 53299d8..5188a43 100644
--- a/hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/protocols/muxdemux/IBufferFactory.java
+++ b/hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/IBufferFactory.java
@@ -16,7 +16,7 @@
  * specific language governing permissions and limitations
  * under the License.
  */
-package org.apache.hyracks.net.protocols.muxdemux;
+package org.apache.hyracks.api.comm;
 
 import java.nio.ByteBuffer;
 
diff --git a/hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/buffers/ICloseableBufferAcceptor.java b/hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/IChannelControlBlock.java
similarity index 62%
copy from hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/buffers/ICloseableBufferAcceptor.java
copy to hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/IChannelControlBlock.java
index 1697cfd..4b327b3 100644
--- a/hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/buffers/ICloseableBufferAcceptor.java
+++ b/hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/IChannelControlBlock.java
@@ -16,25 +16,30 @@
  * specific language governing permissions and limitations
  * under the License.
  */
-package org.apache.hyracks.net.buffers;
+package org.apache.hyracks.api.comm;
 
-/**
- * A buffer acceptor that can be closed to indicate end of transmission or an error code
- * specified to indicate an error in transmission.
- *
- * @author vinayakb
- */
-public interface ICloseableBufferAcceptor extends IBufferAcceptor {
-    /**
-     * Close the buffer acceptor.
-     */
-    public void close();
+public interface IChannelControlBlock {
 
     /**
-     * Indicate that an error occurred.
+     * Get the read interface of this channel.
      *
-     * @param ecode
-     *            - the error code.
+     * @return the read interface.
      */
-    public void error(int ecode);
+    IChannelReadInterface getReadInterface();
+
+    /**
+     * Get the write interface of this channel.
+     *
+     * @return the write interface.
+     */
+    IChannelWriteInterface getWriteInterface();
+
+    /**
+     * Add write credit to this channel.
+     *
+     * @param delta
+     *            number of bytes
+     */
+    public void addWriteCredits(int delta);
+
 }
diff --git a/hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/protocols/muxdemux/IChannelReadInterface.java b/hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/IChannelReadInterface.java
similarity index 77%
rename from hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/protocols/muxdemux/IChannelReadInterface.java
rename to hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/IChannelReadInterface.java
index 8639fb7..86189f1 100644
--- a/hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/protocols/muxdemux/IChannelReadInterface.java
+++ b/hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/IChannelReadInterface.java
@@ -16,13 +16,10 @@
  * specific language governing permissions and limitations
  * under the License.
  */
-package org.apache.hyracks.net.protocols.muxdemux;
-
-import org.apache.hyracks.net.buffers.IBufferAcceptor;
-import org.apache.hyracks.net.buffers.ICloseableBufferAcceptor;
+package org.apache.hyracks.api.comm;
 
 /**
- * Represents the read interface of a {@link ChannelControlBlock}.
+ * Represents the read interface of a {@link IChannelControlBlock}.
  *
  * @author vinayakb
  */
@@ -56,4 +53,19 @@
      *            - the size of each buffer
      */
     public void setBufferFactory(IBufferFactory bufferFactory, int limit, int frameSize);
+
+    /**
+     * Set a flag to indicate whether to flush the current read buffer
+     * on complete reads or wait until the buffer is completely filled
+     *
+     * @param flushOnCompleteRead
+     */
+    public void setFlushOnCompleteRead(boolean flushOnCompleteRead);
+
+    /**
+     * Overrides the default empty buffer acceptor of this {@link IChannelReadInterface}
+     *
+     * @param eba
+     */
+    public void setEmptyBufferAcceptor(IBufferAcceptor eba);
 }
diff --git a/hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/protocols/muxdemux/IChannelWriteInterface.java b/hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/IChannelWriteInterface.java
similarity index 91%
rename from hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/protocols/muxdemux/IChannelWriteInterface.java
rename to hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/IChannelWriteInterface.java
index dc38ea4..6c4ef0e 100644
--- a/hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/protocols/muxdemux/IChannelWriteInterface.java
+++ b/hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/IChannelWriteInterface.java
@@ -16,10 +16,7 @@
  * specific language governing permissions and limitations
  * under the License.
  */
-package org.apache.hyracks.net.protocols.muxdemux;
-
-import org.apache.hyracks.net.buffers.IBufferAcceptor;
-import org.apache.hyracks.net.buffers.ICloseableBufferAcceptor;
+package org.apache.hyracks.api.comm;
 
 /**
  * Represents the write interface of a {@link ChannelControlBlock}.
diff --git a/hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/buffers/ICloseableBufferAcceptor.java b/hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/ICloseableBufferAcceptor.java
similarity index 96%
rename from hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/buffers/ICloseableBufferAcceptor.java
rename to hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/ICloseableBufferAcceptor.java
index 1697cfd..fba0eaf 100644
--- a/hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/buffers/ICloseableBufferAcceptor.java
+++ b/hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/ICloseableBufferAcceptor.java
@@ -16,7 +16,7 @@
  * specific language governing permissions and limitations
  * under the License.
  */
-package org.apache.hyracks.net.buffers;
+package org.apache.hyracks.api.comm;
 
 /**
  * A buffer acceptor that can be closed to indicate end of transmission or an error code
diff --git a/hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/messages/IMessageBroker.java b/hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/messages/IMessageBroker.java
index 2b166d2..7ba9581 100644
--- a/hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/messages/IMessageBroker.java
+++ b/hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/messages/IMessageBroker.java
@@ -18,11 +18,13 @@
  */
 package org.apache.hyracks.api.messages;
 
-/**
- * @author rico
- */
+import org.apache.hyracks.api.comm.IChannelControlBlock;
+
 public interface IMessageBroker {
 
     public void receivedMessage(IMessage message, String nodeId) throws Exception;
 
+    public default void registerMessagingChannel(String nodeId, IChannelControlBlock ccb) {
+        throw new UnsupportedOperationException();
+    }
 }
diff --git a/hyracks-fullstack/hyracks/hyracks-comm/src/main/java/org/apache/hyracks/comm/channels/DatasetNetworkInputChannel.java b/hyracks-fullstack/hyracks/hyracks-comm/src/main/java/org/apache/hyracks/comm/channels/DatasetNetworkInputChannel.java
index fba7cf5..8f72bb6 100644
--- a/hyracks-fullstack/hyracks/hyracks-comm/src/main/java/org/apache/hyracks/comm/channels/DatasetNetworkInputChannel.java
+++ b/hyracks-fullstack/hyracks/hyracks-comm/src/main/java/org/apache/hyracks/comm/channels/DatasetNetworkInputChannel.java
@@ -27,12 +27,12 @@
 
 import org.apache.hyracks.api.channels.IInputChannel;
 import org.apache.hyracks.api.channels.IInputChannelMonitor;
+import org.apache.hyracks.api.comm.IBufferAcceptor;
+import org.apache.hyracks.api.comm.ICloseableBufferAcceptor;
 import org.apache.hyracks.api.context.IHyracksCommonContext;
 import org.apache.hyracks.api.dataset.ResultSetId;
 import org.apache.hyracks.api.exceptions.HyracksDataException;
 import org.apache.hyracks.api.job.JobId;
-import org.apache.hyracks.net.buffers.IBufferAcceptor;
-import org.apache.hyracks.net.buffers.ICloseableBufferAcceptor;
 import org.apache.hyracks.net.protocols.muxdemux.ChannelControlBlock;
 
 public class DatasetNetworkInputChannel implements IInputChannel {
diff --git a/hyracks-fullstack/hyracks/hyracks-comm/src/main/java/org/apache/hyracks/comm/channels/NcMessagingUtil.java b/hyracks-fullstack/hyracks/hyracks-comm/src/main/java/org/apache/hyracks/comm/channels/NcMessagingUtil.java
new file mode 100644
index 0000000..17ae249
--- /dev/null
+++ b/hyracks-fullstack/hyracks/hyracks-comm/src/main/java/org/apache/hyracks/comm/channels/NcMessagingUtil.java
@@ -0,0 +1,109 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements.  See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership.  The ASF licenses this file
+ * to you 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.apache.hyracks.comm.channels;
+
+import java.nio.ByteBuffer;
+import java.util.Collections;
+import java.util.HashMap;
+import java.util.Map;
+
+import org.apache.hyracks.api.dataflow.ConnectorDescriptorId;
+import org.apache.hyracks.api.job.JobId;
+import org.apache.hyracks.api.partitions.PartitionId;
+import org.apache.hyracks.util.StorageUtil;
+import org.apache.hyracks.util.StorageUtil.StorageUnit;
+
+public class NcMessagingUtil {
+
+    public static final int MAX_INITIAL_MESSAGE_SIZE = StorageUtil.getSizeInBytes(2, StorageUnit.KILOBYTE);
+
+    private NcMessagingUtil() {
+        throw new AssertionError("This util class should not be initialized");
+    }
+
+    public enum ChannelType {
+        DATA_PARTITION((byte) 0, 21),
+        APP_MESSAGING((byte) 1, 5);
+
+        private static final Map<Byte, ChannelType> TYPES = Collections.unmodifiableMap(initializeMapping());
+        private final int initialMessageBaseSize;
+        private final byte type;
+
+        ChannelType(byte type, int initialMessageBaseSize) {
+            this.type = type;
+            this.initialMessageBaseSize = initialMessageBaseSize;
+        }
+
+        public byte getType() {
+            return type;
+        }
+
+        public int getInitialMessageBaseSize() {
+            return initialMessageBaseSize;
+        }
+
+        public static ChannelType getChannelType(byte type) {
+            return TYPES.get(type);
+        }
+
+        private static Map<Byte, ChannelType> initializeMapping() {
+            Map<Byte, ChannelType> mapping = new HashMap<>();
+            for (ChannelType s : ChannelType.values()) {
+                mapping.put(s.getType(), s);
+            }
+            return mapping;
+        }
+    }
+
+    public static ByteBuffer getDataPartitionChannelInitialBuffer(PartitionId partitionId) {
+        ByteBuffer initialBuffer = ByteBuffer.allocate(ChannelType.DATA_PARTITION.getInitialMessageBaseSize());
+        initialBuffer.put(ChannelType.DATA_PARTITION.getType());
+        initialBuffer.putLong(partitionId.getJobId().getId());
+        initialBuffer.putInt(partitionId.getConnectorDescriptorId().getId());
+        initialBuffer.putInt(partitionId.getSenderIndex());
+        initialBuffer.putInt(partitionId.getReceiverIndex());
+        initialBuffer.flip();
+        return initialBuffer;
+    }
+
+    public static ByteBuffer getMessagingChannelInitialBuffer(String nodeId) {
+        ByteBuffer initialBuffer = ByteBuffer
+                .allocate(ChannelType.APP_MESSAGING.getInitialMessageBaseSize() + nodeId.length());
+        initialBuffer.put(ChannelType.APP_MESSAGING.getType());
+        initialBuffer.putInt(nodeId.length());
+        initialBuffer.put(nodeId.getBytes());
+        initialBuffer.flip();
+        return initialBuffer;
+    }
+
+    public static PartitionId readDataPartitionInitialMessage(ByteBuffer buffer) {
+        JobId jobId = new JobId(buffer.getLong());
+        ConnectorDescriptorId cdid = new ConnectorDescriptorId(buffer.getInt());
+        int senderIndex = buffer.getInt();
+        int receiverIndex = buffer.getInt();
+        return new PartitionId(jobId, cdid, senderIndex, receiverIndex);
+    }
+
+    public static String readAppMessagingInitialMessage(ByteBuffer buffer) {
+        int nodeIdLength = buffer.getInt();
+        byte[] stringBytes = new byte[nodeIdLength];
+        buffer.get(stringBytes);
+        return new String(stringBytes);
+    }
+}
diff --git a/hyracks-fullstack/hyracks/hyracks-comm/src/main/java/org/apache/hyracks/comm/channels/NetworkInputChannel.java b/hyracks-fullstack/hyracks/hyracks-comm/src/main/java/org/apache/hyracks/comm/channels/NetworkInputChannel.java
index 235536f..7938fc1 100644
--- a/hyracks-fullstack/hyracks/hyracks-comm/src/main/java/org/apache/hyracks/comm/channels/NetworkInputChannel.java
+++ b/hyracks-fullstack/hyracks/hyracks-comm/src/main/java/org/apache/hyracks/comm/channels/NetworkInputChannel.java
@@ -27,11 +27,11 @@
 
 import org.apache.hyracks.api.channels.IInputChannel;
 import org.apache.hyracks.api.channels.IInputChannelMonitor;
+import org.apache.hyracks.api.comm.IBufferAcceptor;
+import org.apache.hyracks.api.comm.ICloseableBufferAcceptor;
 import org.apache.hyracks.api.context.IHyracksCommonContext;
 import org.apache.hyracks.api.exceptions.HyracksDataException;
 import org.apache.hyracks.api.partitions.PartitionId;
-import org.apache.hyracks.net.buffers.IBufferAcceptor;
-import org.apache.hyracks.net.buffers.ICloseableBufferAcceptor;
 import org.apache.hyracks.net.protocols.muxdemux.ChannelControlBlock;
 
 public class NetworkInputChannel implements IInputChannel {
diff --git a/hyracks-fullstack/hyracks/hyracks-comm/src/main/java/org/apache/hyracks/comm/channels/NetworkOutputChannel.java b/hyracks-fullstack/hyracks/hyracks-comm/src/main/java/org/apache/hyracks/comm/channels/NetworkOutputChannel.java
index 46220de..60e2e35 100644
--- a/hyracks-fullstack/hyracks/hyracks-comm/src/main/java/org/apache/hyracks/comm/channels/NetworkOutputChannel.java
+++ b/hyracks-fullstack/hyracks/hyracks-comm/src/main/java/org/apache/hyracks/comm/channels/NetworkOutputChannel.java
@@ -22,9 +22,9 @@
 import java.util.ArrayDeque;
 import java.util.Deque;
 
+import org.apache.hyracks.api.comm.IBufferAcceptor;
 import org.apache.hyracks.api.comm.IFrameWriter;
 import org.apache.hyracks.api.exceptions.HyracksDataException;
-import org.apache.hyracks.net.buffers.IBufferAcceptor;
 import org.apache.hyracks.net.protocols.muxdemux.ChannelControlBlock;
 
 public class NetworkOutputChannel implements IFrameWriter {
diff --git a/hyracks-fullstack/hyracks/hyracks-comm/src/main/java/org/apache/hyracks/comm/channels/ReadBufferFactory.java b/hyracks-fullstack/hyracks/hyracks-comm/src/main/java/org/apache/hyracks/comm/channels/ReadBufferFactory.java
index 3aa77b9..7628cd2 100644
--- a/hyracks-fullstack/hyracks/hyracks-comm/src/main/java/org/apache/hyracks/comm/channels/ReadBufferFactory.java
+++ b/hyracks-fullstack/hyracks/hyracks-comm/src/main/java/org/apache/hyracks/comm/channels/ReadBufferFactory.java
@@ -20,8 +20,8 @@
 
 import java.nio.ByteBuffer;
 
+import org.apache.hyracks.api.comm.IBufferFactory;
 import org.apache.hyracks.api.context.IHyracksCommonContext;
-import org.apache.hyracks.net.protocols.muxdemux.IBufferFactory;
 
 /**
  * @author yingyib
diff --git a/hyracks-fullstack/hyracks/hyracks-control/hyracks-control-cc/src/main/java/org/apache/hyracks/control/cc/NodeControllerState.java b/hyracks-fullstack/hyracks/hyracks-control/hyracks-control-cc/src/main/java/org/apache/hyracks/control/cc/NodeControllerState.java
index d93ba0d..ff5832a 100644
--- a/hyracks-fullstack/hyracks/hyracks-control/hyracks-control-cc/src/main/java/org/apache/hyracks/control/cc/NodeControllerState.java
+++ b/hyracks-fullstack/hyracks/hyracks-control/hyracks-control-cc/src/main/java/org/apache/hyracks/control/cc/NodeControllerState.java
@@ -23,10 +23,6 @@
 import java.util.Map;
 import java.util.Set;
 
-import org.json.JSONArray;
-import org.json.JSONException;
-import org.json.JSONObject;
-
 import org.apache.hyracks.api.comm.NetworkAddress;
 import org.apache.hyracks.api.job.JobId;
 import org.apache.hyracks.control.common.base.INodeController;
@@ -35,6 +31,9 @@
 import org.apache.hyracks.control.common.heartbeat.HeartbeatData;
 import org.apache.hyracks.control.common.heartbeat.HeartbeatSchema;
 import org.apache.hyracks.control.common.heartbeat.HeartbeatSchema.GarbageCollectorInfo;
+import org.json.JSONArray;
+import org.json.JSONException;
+import org.json.JSONObject;
 
 public class NodeControllerState {
     private static final int RRD_SIZE = 720;
@@ -46,6 +45,8 @@
     private final NetworkAddress dataPort;
 
     private final NetworkAddress datasetPort;
+
+    private final NetworkAddress messagingPort;
 
     private final Set<JobId> activeJobIds;
 
@@ -142,6 +143,7 @@
         ncConfig = reg.getNCConfig();
         dataPort = reg.getDataPort();
         datasetPort = reg.getDatasetPort();
+        messagingPort = reg.getMessagingPort();
         activeJobIds = new HashSet<JobId>();
 
         osName = reg.getOSName();
@@ -264,6 +266,10 @@
         return datasetPort;
     }
 
+    public NetworkAddress getMessagingPort() {
+        return messagingPort;
+    }
+
     public JSONObject toSummaryJSON() throws JSONException {
         JSONObject o = new JSONObject();
         o.put("node-id", ncConfig.nodeId);
diff --git a/hyracks-fullstack/hyracks/hyracks-control/hyracks-control-cc/src/main/java/org/apache/hyracks/control/cc/work/GetNodeControllersInfoWork.java b/hyracks-fullstack/hyracks/hyracks-control/hyracks-control-cc/src/main/java/org/apache/hyracks/control/cc/work/GetNodeControllersInfoWork.java
index 785a202..726bf12 100644
--- a/hyracks-fullstack/hyracks/hyracks-control/hyracks-control-cc/src/main/java/org/apache/hyracks/control/cc/work/GetNodeControllersInfoWork.java
+++ b/hyracks-fullstack/hyracks/hyracks-control/hyracks-control-cc/src/main/java/org/apache/hyracks/control/cc/work/GetNodeControllersInfoWork.java
@@ -40,11 +40,11 @@
 
     @Override
     public void run() {
-        Map<String, NodeControllerInfo> result = new LinkedHashMap<String, NodeControllerInfo>();
+        Map<String, NodeControllerInfo> result = new LinkedHashMap<>();
         Map<String, NodeControllerState> nodeMap = ccs.getNodeMap();
         for (Map.Entry<String, NodeControllerState> e : nodeMap.entrySet()) {
-            result.put(e.getKey(), new NodeControllerInfo(e.getKey(), NodeStatus.ALIVE, e.getValue().getDataPort(), e
-                    .getValue().getDatasetPort()));
+            result.put(e.getKey(), new NodeControllerInfo(e.getKey(), NodeStatus.ALIVE, e.getValue().getDataPort(),
+                    e.getValue().getDatasetPort(), e.getValue().getMessagingPort()));
         }
         callback.setValue(result);
     }
diff --git a/hyracks-fullstack/hyracks/hyracks-control/hyracks-control-common/src/main/java/org/apache/hyracks/control/common/controllers/NCConfig.java b/hyracks-fullstack/hyracks/hyracks-control/hyracks-control-common/src/main/java/org/apache/hyracks/control/common/controllers/NCConfig.java
index b408083..6eb5264 100644
--- a/hyracks-fullstack/hyracks/hyracks-control/hyracks-control-common/src/main/java/org/apache/hyracks/control/common/controllers/NCConfig.java
+++ b/hyracks-fullstack/hyracks/hyracks-control/hyracks-control-common/src/main/java/org/apache/hyracks/control/common/controllers/NCConfig.java
@@ -18,18 +18,18 @@
  */
 package org.apache.hyracks.control.common.controllers;
 
+import java.io.IOException;
+import java.io.Serializable;
+import java.net.InetAddress;
+import java.util.List;
+import java.util.Map;
+
 import org.apache.hyracks.api.application.IApplicationConfig;
 import org.apache.hyracks.control.common.application.IniApplicationConfig;
 import org.ini4j.Ini;
 import org.kohsuke.args4j.Argument;
 import org.kohsuke.args4j.Option;
 import org.kohsuke.args4j.spi.StopOptionHandler;
-
-import java.io.IOException;
-import java.io.Serializable;
-import java.net.InetAddress;
-import java.util.List;
-import java.util.Map;
 
 public class NCConfig implements Serializable {
     private static final long serialVersionUID = 2L;
@@ -82,7 +82,7 @@
     @Option(name = "-result-public-port", usage = "Public IP port to announce dataset result distribution listener (default: same as -result-port; must set -result-public-ip-address also)", required = false)
     public int resultPublicPort = 0;
 
-    @Option(name = "-retries", usage ="Number of attempts to contact CC before giving up (default = 5)")
+    @Option(name = "-retries", usage = "Number of attempts to contact CC before giving up (default = 5)")
     public int retries = 5;
 
     @Option(name = "-iodevices", usage = "Comma separated list of IO Device mount points (default: One device in default temp folder)", required = false)
@@ -112,6 +112,19 @@
     @Option(name = "-config-file", usage = "Specify path to local configuration file (default: no local config)", required = false)
     public String configFile = null;
 
+    //TODO add messaging values to NC start scripts
+    @Option(name = "-messaging-ip-address", usage = "IP Address to bind messaging listener (default: same as -address)", required = false)
+    public String messagingIPAddress;
+
+    @Option(name = "-messaging-port", usage = "IP port to bind messaging listener (default: random port)", required = false)
+    public int messagingPort = 0;
+
+    @Option(name = "-messaging-public-ip-address", usage = "Public IP Address to announce messaging listener (default: same as -messaging-ip-address)", required = false)
+    public String messagingPublicIPAddress;
+
+    @Option(name = "-messaging-public-port", usage = "Public IP port to announce messaging listener (default: same as -messaging-port; must set -messaging-public-port also)", required = false)
+    public int messagingPublicPort = 0;
+
     @Argument
     @Option(name = "--", handler = StopOptionHandler.class)
     public List<String> appArgs;
@@ -139,13 +152,14 @@
         resultIPAddress = IniUtils.getString(ini, nodeSection, "result.address", resultIPAddress);
         resultPort = IniUtils.getInt(ini, nodeSection, "result.port", resultPort);
 
-        clusterNetPublicIPAddress = IniUtils.getString(
-                ini, nodeSection, "public.cluster.address", clusterNetPublicIPAddress);
+        clusterNetPublicIPAddress = IniUtils.getString(ini, nodeSection, "public.cluster.address",
+                clusterNetPublicIPAddress);
         clusterNetPublicPort = IniUtils.getInt(ini, nodeSection, "public.cluster.port", clusterNetPublicPort);
         dataPublicIPAddress = IniUtils.getString(ini, nodeSection, "public.data.address", dataPublicIPAddress);
         dataPublicPort = IniUtils.getInt(ini, nodeSection, "public.data.port", dataPublicPort);
         resultPublicIPAddress = IniUtils.getString(ini, nodeSection, "public.result.address", resultPublicIPAddress);
         resultPublicPort = IniUtils.getInt(ini, nodeSection, "public.result.port", resultPublicPort);
+        //TODO pass messaging info from ini file
 
         retries = IniUtils.getInt(ini, nodeSection, "retries", retries);
 
@@ -169,23 +183,32 @@
         }
 
         // "address" is the default for all IP addresses
-        if (clusterNetIPAddress == null) clusterNetIPAddress = ipAddress;
-        if (dataIPAddress == null) dataIPAddress = ipAddress;
-        if (resultIPAddress == null) resultIPAddress = ipAddress;
+        if (clusterNetIPAddress == null)
+            clusterNetIPAddress = ipAddress;
+        if (dataIPAddress == null)
+            dataIPAddress = ipAddress;
+        if (resultIPAddress == null)
+            resultIPAddress = ipAddress;
 
         // All "public" options default to their "non-public" versions
-        if (clusterNetPublicIPAddress == null) clusterNetPublicIPAddress = clusterNetIPAddress;
-        if (clusterNetPublicPort == 0) clusterNetPublicPort = clusterNetPort;
-        if (dataPublicIPAddress == null) dataPublicIPAddress = dataIPAddress;
-        if (dataPublicPort == 0) dataPublicPort = dataPort;
-        if (resultPublicIPAddress == null) resultPublicIPAddress = resultIPAddress;
-        if (resultPublicPort == 0) resultPublicPort = resultPort;
+        if (clusterNetPublicIPAddress == null)
+            clusterNetPublicIPAddress = clusterNetIPAddress;
+        if (clusterNetPublicPort == 0)
+            clusterNetPublicPort = clusterNetPort;
+        if (dataPublicIPAddress == null)
+            dataPublicIPAddress = dataIPAddress;
+        if (dataPublicPort == 0)
+            dataPublicPort = dataPort;
+        if (resultPublicIPAddress == null)
+            resultPublicIPAddress = resultIPAddress;
+        if (resultPublicPort == 0)
+            resultPublicPort = resultPort;
     }
 
     /**
      * @return An IApplicationConfig representing this NCConfig.
-     * Note: Currently this only includes the values from the configuration
-     * file, not anything specified on the command-line. QQQ
+     *         Note: Currently this only includes the values from the configuration
+     *         file, not anything specified on the command-line. QQQ
      */
     public IApplicationConfig getAppConfig() {
         return new IniApplicationConfig(ini);
@@ -222,6 +245,14 @@
         cList.add(resultPublicIPAddress);
         cList.add("-result-public-port");
         cList.add(String.valueOf(resultPublicPort));
+        cList.add("-messaging-ip-address");
+        cList.add(messagingIPAddress);
+        cList.add("-messaging-port");
+        cList.add(String.valueOf(messagingPort));
+        cList.add("-messaging-public-ip-address");
+        cList.add(messagingPublicIPAddress);
+        cList.add("-messaging-public-port");
+        cList.add(String.valueOf(messagingPublicPort));
         cList.add("-retries");
         cList.add(String.valueOf(retries));
         cList.add("-iodevices");
@@ -275,7 +306,10 @@
         configuration.put("result-time-to-live", String.valueOf(resultTTL));
         configuration.put("result-sweep-threshold", String.valueOf(resultSweepThreshold));
         configuration.put("result-manager-memory", String.valueOf(resultManagerMemory));
-
+        configuration.put("messaging-ip-address", messagingIPAddress);
+        configuration.put("messaging-port", String.valueOf(messagingPort));
+        configuration.put("messaging-public-ip-address", messagingPublicIPAddress);
+        configuration.put("messaging-public-port", String.valueOf(messagingPublicPort));
         if (appNCMainClass != null) {
             configuration.put("app-nc-main-class", appNCMainClass);
         }
diff --git a/hyracks-fullstack/hyracks/hyracks-control/hyracks-control-common/src/main/java/org/apache/hyracks/control/common/controllers/NodeRegistration.java b/hyracks-fullstack/hyracks/hyracks-control/hyracks-control-common/src/main/java/org/apache/hyracks/control/common/controllers/NodeRegistration.java
index 5a23455..bb8022e 100644
--- a/hyracks-fullstack/hyracks/hyracks-control/hyracks-control-common/src/main/java/org/apache/hyracks/control/common/controllers/NodeRegistration.java
+++ b/hyracks-fullstack/hyracks/hyracks-control/hyracks-control-common/src/main/java/org/apache/hyracks/control/common/controllers/NodeRegistration.java
@@ -65,10 +65,13 @@
 
     private final HeartbeatSchema hbSchema;
 
+    private final NetworkAddress messagingPort;
+
     public NodeRegistration(InetSocketAddress ncAddress, String nodeId, NCConfig ncConfig, NetworkAddress dataPort,
             NetworkAddress datasetPort, String osName, String arch, String osVersion, int nProcessors, String vmName,
             String vmVersion, String vmVendor, String classpath, String libraryPath, String bootClasspath,
-            List<String> inputArguments, Map<String, String> systemProperties, HeartbeatSchema hbSchema) {
+            List<String> inputArguments, Map<String, String> systemProperties, HeartbeatSchema hbSchema,
+            NetworkAddress messagingPort) {
         this.ncAddress = ncAddress;
         this.nodeId = nodeId;
         this.ncConfig = ncConfig;
@@ -87,6 +90,7 @@
         this.inputArguments = inputArguments;
         this.systemProperties = systemProperties;
         this.hbSchema = hbSchema;
+        this.messagingPort = messagingPort;
     }
 
     public InetSocketAddress getNodeControllerAddress() {
@@ -160,4 +164,8 @@
     public Map<String, String> getSystemProperties() {
         return systemProperties;
     }
+
+    public NetworkAddress getMessagingPort() {
+        return messagingPort;
+    }
 }
diff --git a/hyracks-fullstack/hyracks/hyracks-control/hyracks-control-nc/src/main/java/org/apache/hyracks/control/nc/NodeControllerService.java b/hyracks-fullstack/hyracks/hyracks-control/hyracks-control-nc/src/main/java/org/apache/hyracks/control/nc/NodeControllerService.java
index 2f8def1..620f3ca 100644
--- a/hyracks-fullstack/hyracks/hyracks-control/hyracks-control-nc/src/main/java/org/apache/hyracks/control/nc/NodeControllerService.java
+++ b/hyracks-fullstack/hyracks/hyracks-control/hyracks-control-nc/src/main/java/org/apache/hyracks/control/nc/NodeControllerService.java
@@ -73,6 +73,7 @@
 import org.apache.hyracks.control.nc.io.profiling.IIOCounter;
 import org.apache.hyracks.control.nc.io.profiling.IOCounterFactory;
 import org.apache.hyracks.control.nc.net.DatasetNetworkManager;
+import org.apache.hyracks.control.nc.net.MessagingNetworkManager;
 import org.apache.hyracks.control.nc.net.NetworkManager;
 import org.apache.hyracks.control.nc.partitions.PartitionManager;
 import org.apache.hyracks.control.nc.resources.memory.MemoryManager;
@@ -157,6 +158,8 @@
     private boolean shuttedDown = false;
 
     private IIOCounter ioCounter;
+
+    private MessagingNetworkManager messagingNetManager;
 
     public NodeControllerService(NCConfig ncConfig) throws Exception {
         this.ncConfig = ncConfig;
@@ -248,6 +251,9 @@
         datasetNetworkManager = new DatasetNetworkManager(ncConfig.resultIPAddress, ncConfig.resultPort,
                 datasetPartitionManager, ncConfig.nNetThreads, ncConfig.nNetBuffers, ncConfig.resultPublicIPAddress,
                 ncConfig.resultPublicPort);
+        messagingNetManager = new MessagingNetworkManager(appCtx.getMessageBroker(), ncConfig.messagingIPAddress,
+                ncConfig.messagingPort, ncConfig.nNetThreads, ncConfig.messagingPublicIPAddress,
+                ncConfig.messagingPublicPort);
     }
 
     @Override
@@ -260,7 +266,9 @@
         init();
 
         datasetNetworkManager.start();
-        IIPCHandle ccIPCHandle = ipc.getHandle(new InetSocketAddress(ncConfig.ccHost, ncConfig.ccPort), ncConfig.retries);
+        messagingNetManager.start();
+        IIPCHandle ccIPCHandle = ipc.getHandle(new InetSocketAddress(ncConfig.ccHost, ncConfig.ccPort),
+                ncConfig.retries);
         this.ccs = new ClusterControllerRemoteProxy(ccIPCHandle);
         HeartbeatSchema.GarbageCollectorInfo[] gcInfos = new HeartbeatSchema.GarbageCollectorInfo[gcMXBeans.size()];
         for (int i = 0; i < gcInfos.length; ++i) {
@@ -270,12 +278,12 @@
         // Use "public" versions of network addresses and ports
         NetworkAddress datasetAddress = datasetNetworkManager.getPublicNetworkAddress();
         NetworkAddress netAddress = netManager.getPublicNetworkAddress();
-        ccs.registerNode(new NodeRegistration(ipc.getSocketAddress(), id, ncConfig, netAddress,
-                datasetAddress, osMXBean.getName(), osMXBean.getArch(), osMXBean
-                        .getVersion(), osMXBean.getAvailableProcessors(), runtimeMXBean.getVmName(), runtimeMXBean
-                        .getVmVersion(), runtimeMXBean.getVmVendor(), runtimeMXBean.getClassPath(), runtimeMXBean
-                        .getLibraryPath(), runtimeMXBean.getBootClassPath(), runtimeMXBean.getInputArguments(),
-                runtimeMXBean.getSystemProperties(), hbSchema));
+        NetworkAddress meesagingPort = messagingNetManager.getPublicNetworkAddress();
+        ccs.registerNode(new NodeRegistration(ipc.getSocketAddress(), id, ncConfig, netAddress, datasetAddress,
+                osMXBean.getName(), osMXBean.getArch(), osMXBean.getVersion(), osMXBean.getAvailableProcessors(),
+                runtimeMXBean.getVmName(), runtimeMXBean.getVmVersion(), runtimeMXBean.getVmVendor(),
+                runtimeMXBean.getClassPath(), runtimeMXBean.getLibraryPath(), runtimeMXBean.getBootClassPath(),
+                runtimeMXBean.getInputArguments(), runtimeMXBean.getSystemProperties(), hbSchema, meesagingPort));
 
         synchronized (this) {
             while (registrationPending) {
@@ -569,6 +577,10 @@
         return datasetPartitionManager;
     }
 
+    public MessagingNetworkManager getMessagingNetManager() {
+        return messagingNetManager;
+    }
+
     /**
      * Shutdown hook that invokes {@link NCApplicationEntryPoint#stop() stop} method.
      */
diff --git a/hyracks-fullstack/hyracks/hyracks-control/hyracks-control-nc/src/main/java/org/apache/hyracks/control/nc/net/DatasetNetworkManager.java b/hyracks-fullstack/hyracks/hyracks-control/hyracks-control-nc/src/main/java/org/apache/hyracks/control/nc/net/DatasetNetworkManager.java
index 3c99ef3..a7c5f78 100644
--- a/hyracks-fullstack/hyracks/hyracks-control/hyracks-control-nc/src/main/java/org/apache/hyracks/control/nc/net/DatasetNetworkManager.java
+++ b/hyracks-fullstack/hyracks/hyracks-control/hyracks-control-nc/src/main/java/org/apache/hyracks/control/nc/net/DatasetNetworkManager.java
@@ -25,6 +25,7 @@
 import java.util.logging.Level;
 import java.util.logging.Logger;
 
+import org.apache.hyracks.api.comm.ICloseableBufferAcceptor;
 import org.apache.hyracks.api.comm.NetworkAddress;
 import org.apache.hyracks.api.dataset.IDatasetPartitionManager;
 import org.apache.hyracks.api.dataset.ResultSetId;
@@ -32,7 +33,6 @@
 import org.apache.hyracks.api.job.JobId;
 import org.apache.hyracks.comm.channels.IChannelConnectionFactory;
 import org.apache.hyracks.comm.channels.NetworkOutputChannel;
-import org.apache.hyracks.net.buffers.ICloseableBufferAcceptor;
 import org.apache.hyracks.net.exceptions.NetException;
 import org.apache.hyracks.net.protocols.muxdemux.ChannelControlBlock;
 import org.apache.hyracks.net.protocols.muxdemux.IChannelOpenListener;
diff --git a/hyracks-fullstack/hyracks/hyracks-control/hyracks-control-nc/src/main/java/org/apache/hyracks/control/nc/net/MessagingNetworkManager.java b/hyracks-fullstack/hyracks/hyracks-control/hyracks-control-nc/src/main/java/org/apache/hyracks/control/nc/net/MessagingNetworkManager.java
new file mode 100644
index 0000000..a04a939
--- /dev/null
+++ b/hyracks-fullstack/hyracks/hyracks-control/hyracks-control-nc/src/main/java/org/apache/hyracks/control/nc/net/MessagingNetworkManager.java
@@ -0,0 +1,138 @@
+package org.apache.hyracks.control.nc.net;
+
+import java.io.IOException;
+import java.net.InetSocketAddress;
+import java.net.SocketAddress;
+import java.nio.ByteBuffer;
+import java.util.logging.Level;
+import java.util.logging.Logger;
+
+import org.apache.hyracks.api.comm.ICloseableBufferAcceptor;
+import org.apache.hyracks.api.comm.NetworkAddress;
+import org.apache.hyracks.api.exceptions.HyracksDataException;
+import org.apache.hyracks.api.messages.IMessageBroker;
+import org.apache.hyracks.comm.channels.IChannelConnectionFactory;
+import org.apache.hyracks.net.exceptions.NetException;
+import org.apache.hyracks.net.protocols.muxdemux.ChannelControlBlock;
+import org.apache.hyracks.net.protocols.muxdemux.IChannelOpenListener;
+import org.apache.hyracks.net.protocols.muxdemux.MultiplexedConnection;
+import org.apache.hyracks.net.protocols.muxdemux.MuxDemux;
+import org.apache.hyracks.net.protocols.muxdemux.MuxDemuxPerformanceCounters;
+import org.apache.hyracks.util.StorageUtil;
+import org.apache.hyracks.util.StorageUtil.StorageUnit;
+
+public class MessagingNetworkManager implements IChannelConnectionFactory {
+
+    public static final int MAX_INITIAL_MESSAGE_SIZE = StorageUtil.getSizeInBytes(2, StorageUnit.KILOBYTE);
+    private static final Logger LOGGER = Logger.getLogger(MessagingNetworkManager.class.getName());
+    private static final int MAX_CONNECTION_ATTEMPTS = 5;
+    private final MuxDemux md;
+    private final IMessageBroker messageBroker;
+    private NetworkAddress localNetworkAddress;
+    private NetworkAddress publicNetworkAddress;
+
+    public MessagingNetworkManager(IMessageBroker messageBroker, String inetAddress, int inetPort, int nThreads,
+            String publicInetAddress, int publicInetPort) {
+        this.messageBroker = messageBroker;
+        md = new MuxDemux(new InetSocketAddress(inetAddress, inetPort), new ChannelOpenListener(), nThreads,
+                MAX_CONNECTION_ATTEMPTS);
+        publicNetworkAddress = new NetworkAddress(publicInetAddress, publicInetPort);
+    }
+
+    public void start() throws IOException {
+        md.start();
+        InetSocketAddress sockAddr = md.getLocalAddress();
+        localNetworkAddress = new NetworkAddress(sockAddr.getHostString(), sockAddr.getPort());
+
+        // See if the public address was explicitly specified, and if not,
+        // make it a copy of localNetworkAddress
+        if (publicNetworkAddress.getAddress() == null) {
+            publicNetworkAddress = localNetworkAddress;
+        } else {
+            // Likewise for public port
+            if (publicNetworkAddress.getPort() == 0) {
+                publicNetworkAddress = new NetworkAddress(publicNetworkAddress.getAddress(), sockAddr.getPort());
+            }
+        }
+    }
+
+    public void stop() {
+
+    }
+
+    @Override
+    public ChannelControlBlock connect(SocketAddress remoteAddress) throws InterruptedException, NetException {
+        MultiplexedConnection mConn = md.connect((InetSocketAddress) remoteAddress);
+        ChannelControlBlock openChannel = mConn.openChannel();
+        /**
+         * since this is a messaging channel, it might not send complete frames.
+         * Therefore, make it flush on every read message.
+         */
+        openChannel.getReadInterface().setFlushOnCompleteRead(true);
+        return openChannel;
+    }
+
+    public MuxDemuxPerformanceCounters getPerformanceCounters() {
+        return md.getPerformanceCounters();
+    }
+
+    public NetworkAddress getPublicNetworkAddress() {
+        return publicNetworkAddress;
+    }
+
+    public ByteBuffer constructAppMessagingInitialMessage(String ncId) throws HyracksDataException {
+        int intialMsgLength = Integer.BYTES + ncId.length();
+        if (intialMsgLength > MAX_INITIAL_MESSAGE_SIZE) {
+            throw new HyracksDataException(
+                    "Initial message exceded maximum size of " + MAX_INITIAL_MESSAGE_SIZE + " bytes");
+        }
+        ByteBuffer initialBuffer = ByteBuffer.allocate(intialMsgLength);
+        initialBuffer.putInt(ncId.length());
+        initialBuffer.put(ncId.getBytes());
+        initialBuffer.flip();
+        return initialBuffer;
+    }
+
+    private class ChannelOpenListener implements IChannelOpenListener {
+        @Override
+        public void channelOpened(ChannelControlBlock channel) {
+            channel.getReadInterface().setFlushOnCompleteRead(true);
+            channel.getReadInterface().setFullBufferAcceptor(new InitialBufferAcceptor(channel));
+            channel.getReadInterface().getEmptyBufferAcceptor().accept(ByteBuffer.allocate(MAX_INITIAL_MESSAGE_SIZE));
+        }
+    }
+
+    private class InitialBufferAcceptor implements ICloseableBufferAcceptor {
+        private final ChannelControlBlock ccb;
+
+        public InitialBufferAcceptor(ChannelControlBlock ccb) {
+            this.ccb = ccb;
+        }
+
+        @Override
+        public void accept(ByteBuffer buffer) {
+            String nodeId = readAppMessagingInitialMessage(buffer);
+            if (LOGGER.isLoggable(Level.FINE)) {
+                LOGGER.fine("Opened messaging channel with node: " + nodeId);
+            }
+            messageBroker.registerMessagingChannel(nodeId, ccb);
+        }
+
+        @Override
+        public void close() {
+
+        }
+
+        @Override
+        public void error(int ecode) {
+
+        }
+
+        private String readAppMessagingInitialMessage(ByteBuffer buffer) {
+            int nodeIdLength = buffer.getInt();
+            byte[] stringBytes = new byte[nodeIdLength];
+            buffer.get(stringBytes);
+            return new String(stringBytes);
+        }
+    }
+}
diff --git a/hyracks-fullstack/hyracks/hyracks-control/hyracks-control-nc/src/main/java/org/apache/hyracks/control/nc/net/NetworkManager.java b/hyracks-fullstack/hyracks/hyracks-control/hyracks-control-nc/src/main/java/org/apache/hyracks/control/nc/net/NetworkManager.java
index 4b28aef..2f1a7d1 100644
--- a/hyracks-fullstack/hyracks/hyracks-control/hyracks-control-nc/src/main/java/org/apache/hyracks/control/nc/net/NetworkManager.java
+++ b/hyracks-fullstack/hyracks/hyracks-control/hyracks-control-nc/src/main/java/org/apache/hyracks/control/nc/net/NetworkManager.java
@@ -25,6 +25,7 @@
 import java.util.logging.Level;
 import java.util.logging.Logger;
 
+import org.apache.hyracks.api.comm.ICloseableBufferAcceptor;
 import org.apache.hyracks.api.comm.NetworkAddress;
 import org.apache.hyracks.api.dataflow.ConnectorDescriptorId;
 import org.apache.hyracks.api.exceptions.HyracksException;
@@ -33,7 +34,6 @@
 import org.apache.hyracks.comm.channels.IChannelConnectionFactory;
 import org.apache.hyracks.comm.channels.NetworkOutputChannel;
 import org.apache.hyracks.control.nc.partitions.PartitionManager;
-import org.apache.hyracks.net.buffers.ICloseableBufferAcceptor;
 import org.apache.hyracks.net.exceptions.NetException;
 import org.apache.hyracks.net.protocols.muxdemux.ChannelControlBlock;
 import org.apache.hyracks.net.protocols.muxdemux.IChannelOpenListener;
diff --git a/hyracks-fullstack/hyracks/hyracks-hdfs/hyracks-hdfs-core/src/test/java/org/apache/hyracks/hdfs/scheduler/SchedulerTest.java b/hyracks-fullstack/hyracks/hyracks-hdfs/hyracks-hdfs-core/src/test/java/org/apache/hyracks/hdfs/scheduler/SchedulerTest.java
index 373fe21..13cb171 100644
--- a/hyracks-fullstack/hyracks/hyracks-hdfs/hyracks-hdfs-core/src/test/java/org/apache/hyracks/hdfs/scheduler/SchedulerTest.java
+++ b/hyracks-fullstack/hyracks/hyracks-hdfs/hyracks-hdfs-core/src/test/java/org/apache/hyracks/hdfs/scheduler/SchedulerTest.java
@@ -21,24 +21,22 @@
 
 import java.io.FileReader;
 import java.io.IOException;
-import java.net.InetAddress;
-import java.util.HashMap;
 import java.util.Map;
-
-import junit.framework.Assert;
-import junit.framework.TestCase;
 
 import org.apache.hadoop.fs.Path;
 import org.apache.hadoop.mapred.FileSplit;
 import org.apache.hadoop.mapred.InputSplit;
-import org.xml.sax.InputSource;
-import org.xml.sax.SAXException;
-
 import org.apache.hyracks.api.client.NodeControllerInfo;
 import org.apache.hyracks.api.client.NodeStatus;
 import org.apache.hyracks.api.comm.NetworkAddress;
 import org.apache.hyracks.api.topology.ClusterTopology;
 import org.apache.hyracks.api.topology.TopologyDefinitionParser;
+import org.apache.hyracks.hdfs.utils.TestUtils;
+import org.xml.sax.InputSource;
+import org.xml.sax.SAXException;
+
+import junit.framework.Assert;
+import junit.framework.TestCase;
 
 @SuppressWarnings("deprecation")
 public class SchedulerTest extends TestCase {
@@ -60,13 +58,8 @@
      * @throws Exception
      */
     public void testSchedulerSimple() throws Exception {
-        Map<String, NodeControllerInfo> ncNameToNcInfos = new HashMap<String, NodeControllerInfo>();
-        ncNameToNcInfos.put("nc1", new NodeControllerInfo("nc1", NodeStatus.ALIVE, new NetworkAddress("10.0.0.1", 5099), new NetworkAddress("10.0.0.1", 5098)));
-        ncNameToNcInfos.put("nc2", new NodeControllerInfo("nc2", NodeStatus.ALIVE, new NetworkAddress("10.0.0.2", 5099), new NetworkAddress("10.0.0.2", 5098)));
-        ncNameToNcInfos.put("nc3", new NodeControllerInfo("nc3", NodeStatus.ALIVE, new NetworkAddress("10.0.0.3", 5099), new NetworkAddress("10.0.0.3", 5098)));
-        ncNameToNcInfos.put("nc4", new NodeControllerInfo("nc4", NodeStatus.ALIVE, new NetworkAddress("10.0.0.4", 5099), new NetworkAddress("10.0.0.4", 5098)));
-        ncNameToNcInfos.put("nc5", new NodeControllerInfo("nc5", NodeStatus.ALIVE, new NetworkAddress("10.0.0.5", 5099), new NetworkAddress("10.0.0.5", 5098)));
-        ncNameToNcInfos.put("nc6", new NodeControllerInfo("nc6", NodeStatus.ALIVE, new NetworkAddress("10.0.0.6", 5099), new NetworkAddress("10.0.0.6", 5098)));
+        Map<String, NodeControllerInfo> ncNameToNcInfos = TestUtils.generateNodeControllerInfo(6, "nc", "10.0.0.", 5099,
+                5098, 5097);
 
         InputSplit[] fileSplits = new InputSplit[6];
         fileSplits[0] = new FileSplit(new Path("part-1"), 0, 0, new String[] { "10.0.0.1", "10.0.0.2", "10.0.0.3" });
@@ -98,13 +91,17 @@
      * @throws Exception
      */
     public void testSchedulerLargerHDFS() throws Exception {
-        Map<String, NodeControllerInfo> ncNameToNcInfos = new HashMap<String, NodeControllerInfo>();
-        ncNameToNcInfos.put("nc1", new NodeControllerInfo("nc1", NodeStatus.ALIVE, new NetworkAddress("10.0.0.1", 5099), new NetworkAddress("10.0.0.1", 5098)));
-        ncNameToNcInfos.put("nc2", new NodeControllerInfo("nc2", NodeStatus.ALIVE, new NetworkAddress("10.0.0.2", 5099), new NetworkAddress("10.0.0.2", 5098)));
-        ncNameToNcInfos.put("nc3", new NodeControllerInfo("nc3", NodeStatus.ALIVE, new NetworkAddress("10.0.0.3", 5099), new NetworkAddress("10.0.0.3", 5098)));
-        ncNameToNcInfos.put("nc4", new NodeControllerInfo("nc4", NodeStatus.ALIVE, new NetworkAddress("10.0.0.4", 5099), new NetworkAddress("10.0.0.4", 5098)));
-        ncNameToNcInfos.put("nc7", new NodeControllerInfo("nc7", NodeStatus.ALIVE, new NetworkAddress("10.0.0.7", 5099), new NetworkAddress("10.0.0.5", 5098)));
-        ncNameToNcInfos.put("nc12", new NodeControllerInfo("nc12", NodeStatus.ALIVE, new NetworkAddress("10.0.0.12", 5099), new NetworkAddress("10.0.0.5", 5098)));
+        int dataPort = 5099;
+        int resultPort = 5098;
+        int messagingPort = 5097;
+        Map<String, NodeControllerInfo> ncNameToNcInfos = TestUtils.generateNodeControllerInfo(4, "nc", "10.0.0.",
+                dataPort, resultPort, messagingPort);
+        ncNameToNcInfos.put("nc7",
+                new NodeControllerInfo("nc7", NodeStatus.ALIVE, new NetworkAddress("10.0.0.7", dataPort),
+                        new NetworkAddress("10.0.0.5", resultPort), new NetworkAddress("10.0.0.5", messagingPort)));
+        ncNameToNcInfos.put("nc12",
+                new NodeControllerInfo("nc12", NodeStatus.ALIVE, new NetworkAddress("10.0.0.12", dataPort),
+                        new NetworkAddress("10.0.0.5", resultPort), new NetworkAddress("10.0.0.5", messagingPort)));
 
         InputSplit[] fileSplits = new InputSplit[12];
         fileSplits[0] = new FileSplit(new Path("part-1"), 0, 0, new String[] { "10.0.0.1", "10.0.0.2", "10.0.0.3" });
@@ -115,7 +112,8 @@
         fileSplits[5] = new FileSplit(new Path("part-6"), 0, 0, new String[] { "10.0.0.2", "10.0.0.3", "10.0.0.5" });
         fileSplits[6] = new FileSplit(new Path("part-7"), 0, 0, new String[] { "10.0.0.1", "10.0.0.2", "10.0.0.3" });
         fileSplits[7] = new FileSplit(new Path("part-8"), 0, 0, new String[] { "10.0.0.3", "10.0.0.4", "10.0.0.5" });
-        fileSplits[8] = new FileSplit(new Path("part-12"), 0, 0, new String[] { "10.0.0.14", "10.0.0.11", "10.0.0.13" });
+        fileSplits[8] = new FileSplit(new Path("part-12"), 0, 0,
+                new String[] { "10.0.0.14", "10.0.0.11", "10.0.0.13" });
         fileSplits[9] = new FileSplit(new Path("part-10"), 0, 0, new String[] { "10.0.0.2", "10.0.0.1", "10.0.0.6" });
         fileSplits[10] = new FileSplit(new Path("part-11"), 0, 0, new String[] { "10.0.0.3", "10.0.0.4", "10.0.0.7" });
         fileSplits[11] = new FileSplit(new Path("part-9"), 0, 0, new String[] { "10.0.0.4", "10.0.0.5", "10.0.0.6" });
@@ -123,8 +121,8 @@
         Scheduler scheduler = new Scheduler(ncNameToNcInfos);
         String[] locationConstraints = scheduler.getLocationConstraints(fileSplits);
 
-        String[] expectedResults = new String[] { "nc1", "nc4", "nc4", "nc1", "nc3", "nc2", "nc2", "nc3", "nc12",
-                "nc7", "nc7", "nc12" };
+        String[] expectedResults = new String[] { "nc1", "nc4", "nc4", "nc1", "nc3", "nc2", "nc2", "nc3", "nc12", "nc7",
+                "nc7", "nc12" };
         for (int i = 0; i < locationConstraints.length; i++) {
             Assert.assertEquals(locationConstraints[i], expectedResults[i]);
         }
@@ -145,13 +143,8 @@
      * @throws Exception
      */
     public void testSchedulerSmallerHDFS() throws Exception {
-        Map<String, NodeControllerInfo> ncNameToNcInfos = new HashMap<String, NodeControllerInfo>();
-        ncNameToNcInfos.put("nc1", new NodeControllerInfo("nc1", NodeStatus.ALIVE, new NetworkAddress("10.0.0.1", 5099), new NetworkAddress("10.0.0.1", 5098)));
-        ncNameToNcInfos.put("nc2", new NodeControllerInfo("nc2", NodeStatus.ALIVE, new NetworkAddress("10.0.0.2", 5099), new NetworkAddress("10.0.0.2", 5098)));
-        ncNameToNcInfos.put("nc3", new NodeControllerInfo("nc3", NodeStatus.ALIVE, new NetworkAddress("10.0.0.3", 5099), new NetworkAddress("10.0.0.3", 5098)));
-        ncNameToNcInfos.put("nc4", new NodeControllerInfo("nc4", NodeStatus.ALIVE, new NetworkAddress("10.0.0.4", 5099), new NetworkAddress("10.0.0.4", 5098)));
-        ncNameToNcInfos.put("nc5", new NodeControllerInfo("nc5", NodeStatus.ALIVE, new NetworkAddress("10.0.0.5", 5099), new NetworkAddress("10.0.0.5", 5098)));
-        ncNameToNcInfos.put("nc6", new NodeControllerInfo("nc6", NodeStatus.ALIVE, new NetworkAddress("10.0.0.6", 5099), new NetworkAddress("10.0.0.6", 5098)));
+        Map<String, NodeControllerInfo> ncNameToNcInfos = TestUtils.generateNodeControllerInfo(6, "nc", "10.0.0.", 5099,
+                5098, 5097);
 
         InputSplit[] fileSplits = new InputSplit[12];
         fileSplits[0] = new FileSplit(new Path("part-1"), 0, 0, new String[] { "10.0.0.1", "10.0.0.2", "10.0.0.3" });
@@ -191,13 +184,8 @@
      * @throws Exception
      */
     public void testSchedulerSmallerHDFSOdd() throws Exception {
-        Map<String, NodeControllerInfo> ncNameToNcInfos = new HashMap<String, NodeControllerInfo>();
-        ncNameToNcInfos.put("nc1", new NodeControllerInfo("nc1", NodeStatus.ALIVE, new NetworkAddress("10.0.0.1", 5099), new NetworkAddress("10.0.0.1", 5098)));
-        ncNameToNcInfos.put("nc2", new NodeControllerInfo("nc2", NodeStatus.ALIVE, new NetworkAddress("10.0.0.2", 5099), new NetworkAddress("10.0.0.2", 5098)));
-        ncNameToNcInfos.put("nc3", new NodeControllerInfo("nc3", NodeStatus.ALIVE, new NetworkAddress("10.0.0.3", 5099), new NetworkAddress("10.0.0.3", 5098)));
-        ncNameToNcInfos.put("nc4", new NodeControllerInfo("nc4", NodeStatus.ALIVE, new NetworkAddress("10.0.0.4", 5099), new NetworkAddress("10.0.0.4", 5098)));
-        ncNameToNcInfos.put("nc5", new NodeControllerInfo("nc5", NodeStatus.ALIVE, new NetworkAddress("10.0.0.5", 5099), new NetworkAddress("10.0.0.5", 5098)));
-        ncNameToNcInfos.put("nc6", new NodeControllerInfo("nc6", NodeStatus.ALIVE, new NetworkAddress("10.0.0.6", 5099), new NetworkAddress("10.0.0.6", 5098)));
+        Map<String, NodeControllerInfo> ncNameToNcInfos = TestUtils.generateNodeControllerInfo(6, "nc", "10.0.0.", 5099,
+                5098, 5097);
 
         InputSplit[] fileSplits = new InputSplit[13];
         fileSplits[0] = new FileSplit(new Path("part-1"), 0, 0, new String[] { "10.0.0.1", "10.0.0.2", "10.0.0.3" });
@@ -239,13 +227,8 @@
      * @throws Exception
      */
     public void testSchedulercBoundary() throws Exception {
-        Map<String, NodeControllerInfo> ncNameToNcInfos = new HashMap<String, NodeControllerInfo>();
-        ncNameToNcInfos.put("nc1", new NodeControllerInfo("nc1", NodeStatus.ALIVE, new NetworkAddress("10.0.0.1", 5099), new NetworkAddress("10.0.0.1", 5098)));
-        ncNameToNcInfos.put("nc2", new NodeControllerInfo("nc2", NodeStatus.ALIVE, new NetworkAddress("10.0.0.2", 5099), new NetworkAddress("10.0.0.2", 5098)));
-        ncNameToNcInfos.put("nc3", new NodeControllerInfo("nc3", NodeStatus.ALIVE, new NetworkAddress("10.0.0.3", 5099), new NetworkAddress("10.0.0.3", 5098)));
-        ncNameToNcInfos.put("nc4", new NodeControllerInfo("nc4", NodeStatus.ALIVE, new NetworkAddress("10.0.0.4", 5099), new NetworkAddress("10.0.0.4", 5098)));
-        ncNameToNcInfos.put("nc5", new NodeControllerInfo("nc5", NodeStatus.ALIVE, new NetworkAddress("10.0.0.5", 5099), new NetworkAddress("10.0.0.5", 5098)));
-        ncNameToNcInfos.put("nc6", new NodeControllerInfo("nc6", NodeStatus.ALIVE, new NetworkAddress("10.0.0.6", 5099), new NetworkAddress("10.0.0.6", 5098)));
+        Map<String, NodeControllerInfo> ncNameToNcInfos = TestUtils.generateNodeControllerInfo(6, "nc", "10.0.0.", 5099,
+                5098, 5097);
 
         /** test empty file splits */
         InputSplit[] fileSplits = new InputSplit[0];
diff --git a/hyracks-fullstack/hyracks/hyracks-hdfs/hyracks-hdfs-core/src/test/java/org/apache/hyracks/hdfs/utils/TestUtils.java b/hyracks-fullstack/hyracks/hyracks-hdfs/hyracks-hdfs-core/src/test/java/org/apache/hyracks/hdfs/utils/TestUtils.java
index c2ad4a0..8755cf3 100644
--- a/hyracks-fullstack/hyracks/hyracks-hdfs/hyracks-hdfs-core/src/test/java/org/apache/hyracks/hdfs/utils/TestUtils.java
+++ b/hyracks-fullstack/hyracks/hyracks-hdfs/hyracks-hdfs-core/src/test/java/org/apache/hyracks/hdfs/utils/TestUtils.java
@@ -22,6 +22,12 @@
 import java.io.BufferedReader;
 import java.io.File;
 import java.io.FileReader;
+import java.util.HashMap;
+import java.util.Map;
+
+import org.apache.hyracks.api.client.NodeControllerInfo;
+import org.apache.hyracks.api.client.NodeStatus;
+import org.apache.hyracks.api.comm.NetworkAddress;
 
 public class TestUtils {
 
@@ -38,8 +44,8 @@
                     throw new Exception("Actual result changed at line " + num + ":\n< " + lineExpected + "\n> ");
                 }
                 if (!equalStrings(lineExpected, lineActual)) {
-                    throw new Exception("Result for changed at line " + num + ":\n< " + lineExpected + "\n> "
-                            + lineActual);
+                    throw new Exception(
+                            "Result for changed at line " + num + ":\n< " + lineExpected + "\n> " + lineActual);
                 }
                 ++num;
             }
@@ -94,4 +100,16 @@
         return true;
     }
 
+    public static Map<String, NodeControllerInfo> generateNodeControllerInfo(int numberOfNodes, String ncNamePrefix,
+            String addressPrefix, int netPort, int dataPort, int messagingPort) {
+        Map<String, NodeControllerInfo> ncNameToNcInfos = new HashMap<String, NodeControllerInfo>();
+        for (int i = 1; i <= numberOfNodes; i++) {
+            String ncId = ncNamePrefix + i;
+            String ncAddress = addressPrefix + i;
+            ncNameToNcInfos.put(ncId,
+                    new NodeControllerInfo(ncId, NodeStatus.ALIVE, new NetworkAddress(ncAddress, netPort),
+                            new NetworkAddress(ncAddress, dataPort), new NetworkAddress(ncAddress, messagingPort)));
+        }
+        return ncNameToNcInfos;
+    }
 }
diff --git a/hyracks-fullstack/hyracks/hyracks-hdfs/hyracks-hdfs-core/src/test/java/org/apache/hyracks/hdfs2/scheduler/SchedulerTest.java b/hyracks-fullstack/hyracks/hyracks-hdfs/hyracks-hdfs-core/src/test/java/org/apache/hyracks/hdfs2/scheduler/SchedulerTest.java
index 6eabb71..793e029 100644
--- a/hyracks-fullstack/hyracks/hyracks-hdfs/hyracks-hdfs-core/src/test/java/org/apache/hyracks/hdfs2/scheduler/SchedulerTest.java
+++ b/hyracks-fullstack/hyracks/hyracks-hdfs/hyracks-hdfs-core/src/test/java/org/apache/hyracks/hdfs2/scheduler/SchedulerTest.java
@@ -19,22 +19,18 @@
 
 package org.apache.hyracks.hdfs2.scheduler;
 
-import java.net.InetAddress;
 import java.util.ArrayList;
-import java.util.HashMap;
 import java.util.List;
 import java.util.Map;
-
-import junit.framework.Assert;
-import junit.framework.TestCase;
 
 import org.apache.hadoop.fs.Path;
 import org.apache.hadoop.mapreduce.InputSplit;
 import org.apache.hadoop.mapreduce.lib.input.FileSplit;
-
 import org.apache.hyracks.api.client.NodeControllerInfo;
-import org.apache.hyracks.api.client.NodeStatus;
-import org.apache.hyracks.api.comm.NetworkAddress;
+import org.apache.hyracks.hdfs.utils.TestUtils;
+
+import junit.framework.Assert;
+import junit.framework.TestCase;
 
 /**
  * Test case for the new HDFS API scheduler
@@ -47,13 +43,8 @@
      * @throws Exception
      */
     public void testSchedulerSimple() throws Exception {
-        Map<String, NodeControllerInfo> ncNameToNcInfos = new HashMap<String, NodeControllerInfo>();
-        ncNameToNcInfos.put("nc1", new NodeControllerInfo("nc1", NodeStatus.ALIVE, new NetworkAddress("10.0.0.1", 5099), new NetworkAddress("10.0.0.1", 5098)));
-        ncNameToNcInfos.put("nc2", new NodeControllerInfo("nc2", NodeStatus.ALIVE, new NetworkAddress("10.0.0.2", 5099), new NetworkAddress("10.0.0.2", 5098)));
-        ncNameToNcInfos.put("nc3", new NodeControllerInfo("nc3", NodeStatus.ALIVE, new NetworkAddress("10.0.0.3", 5099), new NetworkAddress("10.0.0.3", 5098)));
-        ncNameToNcInfos.put("nc4", new NodeControllerInfo("nc4", NodeStatus.ALIVE, new NetworkAddress("10.0.0.4", 5099), new NetworkAddress("10.0.0.4", 5098)));
-        ncNameToNcInfos.put("nc5", new NodeControllerInfo("nc5", NodeStatus.ALIVE, new NetworkAddress("10.0.0.5", 5099), new NetworkAddress("10.0.0.5", 5098)));
-        ncNameToNcInfos.put("nc6", new NodeControllerInfo("nc6", NodeStatus.ALIVE, new NetworkAddress("10.0.0.6", 5099), new NetworkAddress("10.0.0.6", 5098)));
+        Map<String, NodeControllerInfo> ncNameToNcInfos = TestUtils.generateNodeControllerInfo(6, "nc", "10.0.0.", 5099,
+                5098, 5097);
 
         List<InputSplit> fileSplits = new ArrayList<InputSplit>();
         fileSplits.add(new FileSplit(new Path("part-1"), 0, 0, new String[] { "10.0.0.1", "10.0.0.2", "10.0.0.3" }));
@@ -79,13 +70,8 @@
      * @throws Exception
      */
     public void testSchedulerLargerHDFS() throws Exception {
-        Map<String, NodeControllerInfo> ncNameToNcInfos = new HashMap<String, NodeControllerInfo>();
-        ncNameToNcInfos.put("nc1", new NodeControllerInfo("nc1", NodeStatus.ALIVE, new NetworkAddress("10.0.0.1", 5099), new NetworkAddress("10.0.0.1", 5098)));
-        ncNameToNcInfos.put("nc2", new NodeControllerInfo("nc2", NodeStatus.ALIVE, new NetworkAddress("10.0.0.2", 5099), new NetworkAddress("10.0.0.2", 5098)));
-        ncNameToNcInfos.put("nc3", new NodeControllerInfo("nc3", NodeStatus.ALIVE, new NetworkAddress("10.0.0.3", 5099), new NetworkAddress("10.0.0.3", 5098)));
-        ncNameToNcInfos.put("nc4", new NodeControllerInfo("nc4", NodeStatus.ALIVE, new NetworkAddress("10.0.0.4", 5099), new NetworkAddress("10.0.0.4", 5098)));
-        ncNameToNcInfos.put("nc5", new NodeControllerInfo("nc5", NodeStatus.ALIVE, new NetworkAddress("10.0.0.5", 5099), new NetworkAddress("10.0.0.5", 5098)));
-        ncNameToNcInfos.put("nc6", new NodeControllerInfo("nc6", NodeStatus.ALIVE, new NetworkAddress("10.0.0.6", 5099), new NetworkAddress("10.0.0.6", 5098)));
+        Map<String, NodeControllerInfo> ncNameToNcInfos = TestUtils.generateNodeControllerInfo(6, "nc", "10.0.0.", 5099,
+                5098, 5097);
 
         List<InputSplit> fileSplits = new ArrayList<InputSplit>();
         fileSplits.add(new FileSplit(new Path("part-1"), 0, 0, new String[] { "10.0.0.1", "10.0.0.2", "10.0.0.3" }));
@@ -118,13 +104,8 @@
      * @throws Exception
      */
     public void testSchedulerSmallerHDFS() throws Exception {
-        Map<String, NodeControllerInfo> ncNameToNcInfos = new HashMap<String, NodeControllerInfo>();
-        ncNameToNcInfos.put("nc1", new NodeControllerInfo("nc1", NodeStatus.ALIVE, new NetworkAddress("10.0.0.1", 5099), new NetworkAddress("10.0.0.1", 5098)));
-        ncNameToNcInfos.put("nc2", new NodeControllerInfo("nc2", NodeStatus.ALIVE, new NetworkAddress("10.0.0.2", 5099), new NetworkAddress("10.0.0.2", 5098)));
-        ncNameToNcInfos.put("nc3", new NodeControllerInfo("nc3", NodeStatus.ALIVE, new NetworkAddress("10.0.0.3", 5099), new NetworkAddress("10.0.0.3", 5098)));
-        ncNameToNcInfos.put("nc4", new NodeControllerInfo("nc4", NodeStatus.ALIVE, new NetworkAddress("10.0.0.4", 5099), new NetworkAddress("10.0.0.4", 5098)));
-        ncNameToNcInfos.put("nc5", new NodeControllerInfo("nc5", NodeStatus.ALIVE, new NetworkAddress("10.0.0.5", 5099), new NetworkAddress("10.0.0.5", 5098)));
-        ncNameToNcInfos.put("nc6", new NodeControllerInfo("nc6", NodeStatus.ALIVE, new NetworkAddress("10.0.0.6", 5099), new NetworkAddress("10.0.0.6", 5098)));
+        Map<String, NodeControllerInfo> ncNameToNcInfos = TestUtils.generateNodeControllerInfo(6, "nc", "10.0.0.", 5099,
+                5098, 5097);
 
         List<InputSplit> fileSplits = new ArrayList<InputSplit>();
         fileSplits.add(new FileSplit(new Path("part-1"), 0, 0, new String[] { "10.0.0.1", "10.0.0.2", "10.0.0.3" }));
@@ -157,13 +138,8 @@
      * @throws Exception
      */
     public void testSchedulerSmallerHDFSOdd() throws Exception {
-        Map<String, NodeControllerInfo> ncNameToNcInfos = new HashMap<String, NodeControllerInfo>();
-        ncNameToNcInfos.put("nc1", new NodeControllerInfo("nc1", NodeStatus.ALIVE, new NetworkAddress("10.0.0.1", 5099), new NetworkAddress("10.0.0.1", 5098)));
-        ncNameToNcInfos.put("nc2", new NodeControllerInfo("nc2", NodeStatus.ALIVE, new NetworkAddress("10.0.0.2", 5099), new NetworkAddress("10.0.0.2", 5098)));
-        ncNameToNcInfos.put("nc3", new NodeControllerInfo("nc3", NodeStatus.ALIVE, new NetworkAddress("10.0.0.3", 5099), new NetworkAddress("10.0.0.3", 5098)));
-        ncNameToNcInfos.put("nc4", new NodeControllerInfo("nc4", NodeStatus.ALIVE, new NetworkAddress("10.0.0.4", 5099), new NetworkAddress("10.0.0.4", 5098)));
-        ncNameToNcInfos.put("nc5", new NodeControllerInfo("nc5", NodeStatus.ALIVE, new NetworkAddress("10.0.0.5", 5099), new NetworkAddress("10.0.0.5", 5098)));
-        ncNameToNcInfos.put("nc6", new NodeControllerInfo("nc6", NodeStatus.ALIVE, new NetworkAddress("10.0.0.6", 5099), new NetworkAddress("10.0.0.6", 5098)));
+        Map<String, NodeControllerInfo> ncNameToNcInfos = TestUtils.generateNodeControllerInfo(6, "nc", "10.0.0.", 5099,
+                5098, 5097);
 
         List<InputSplit> fileSplits = new ArrayList<InputSplit>();
         fileSplits.add(new FileSplit(new Path("part-1"), 0, 0, new String[] { "10.0.0.1", "10.0.0.2", "10.0.0.3" }));
diff --git a/hyracks-fullstack/hyracks/hyracks-net/pom.xml b/hyracks-fullstack/hyracks/hyracks-net/pom.xml
index 3c354f0..96b87fb 100644
--- a/hyracks-fullstack/hyracks/hyracks-net/pom.xml
+++ b/hyracks-fullstack/hyracks/hyracks-net/pom.xml
@@ -42,6 +42,11 @@
 
   <dependencies>
     <dependency>
+      <groupId>org.apache.hyracks</groupId>
+      <artifactId>hyracks-api</artifactId>
+      <version>0.2.18-SNAPSHOT</version>
+    </dependency>
+    <dependency>
       <groupId>junit</groupId>
       <artifactId>junit</artifactId>
     </dependency>
diff --git a/hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/protocols/muxdemux/ChannelControlBlock.java b/hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/protocols/muxdemux/ChannelControlBlock.java
index 6d82297..100ed05 100644
--- a/hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/protocols/muxdemux/ChannelControlBlock.java
+++ b/hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/protocols/muxdemux/ChannelControlBlock.java
@@ -28,8 +28,12 @@
 import java.util.logging.Level;
 import java.util.logging.Logger;
 
-import org.apache.hyracks.net.buffers.IBufferAcceptor;
-import org.apache.hyracks.net.buffers.ICloseableBufferAcceptor;
+import org.apache.hyracks.api.comm.IBufferAcceptor;
+import org.apache.hyracks.api.comm.IBufferFactory;
+import org.apache.hyracks.api.comm.IChannelControlBlock;
+import org.apache.hyracks.api.comm.IChannelReadInterface;
+import org.apache.hyracks.api.comm.IChannelWriteInterface;
+import org.apache.hyracks.api.comm.ICloseableBufferAcceptor;
 import org.apache.hyracks.net.exceptions.NetException;
 
 /**
@@ -37,7 +41,7 @@
  *
  * @author vinayakb
  */
-public class ChannelControlBlock {
+public class ChannelControlBlock implements IChannelControlBlock {
     private static final Logger LOGGER = Logger.getLogger(ChannelControlBlock.class.getName());
 
     private final ChannelSet cSet;
@@ -71,20 +75,12 @@
         return channelId;
     }
 
-    /**
-     * Get the read inderface of this channel.
-     *
-     * @return the read interface.
-     */
+    @Override
     public IChannelReadInterface getReadInterface() {
         return ri;
     }
 
-    /**
-     * Get the write interface of this channel.
-     *
-     * @return the write interface.
-     */
+    @Override
     public IChannelWriteInterface getWriteInterface() {
         return wi;
     }
@@ -92,7 +88,7 @@
     private final class ReadInterface implements IChannelReadInterface {
         private final Deque<ByteBuffer> riEmptyStack;
 
-        private final IBufferAcceptor eba = new IBufferAcceptor() {
+        private IBufferAcceptor eba = new IBufferAcceptor() {
             @Override
             public void accept(ByteBuffer buffer) {
                 int delta = buffer.remaining();
@@ -114,6 +110,8 @@
 
         private IBufferFactory bufferFactory;
 
+        private boolean flushOnCompleteRead = false;
+
         ReadInterface() {
             riEmptyStack = new ArrayDeque<ByteBuffer>();
             credits = 0;
@@ -133,6 +131,16 @@
         @Override
         public IBufferAcceptor getEmptyBufferAcceptor() {
             return eba;
+        }
+
+        @Override
+        public void setEmptyBufferAcceptor(IBufferAcceptor eba) {
+            this.eba = eba;
+        }
+
+        @Override
+        public void setFlushOnCompleteRead(boolean flushOnCompleteRead) {
+            this.flushOnCompleteRead = flushOnCompleteRead;
         }
 
         int read(SocketChannel sc, int size) throws IOException, NetException {
@@ -168,7 +176,7 @@
                 } else {
                     return size;
                 }
-                if (currentReadBuffer.remaining() <= 0) {
+                if (currentReadBuffer.remaining() <= 0 || flushOnCompleteRead) {
                     flush();
                 }
             }
@@ -363,7 +371,8 @@
         this.ri.credits = credits;
     }
 
-    synchronized void addWriteCredits(int delta) {
+    @Override
+    public synchronized void addWriteCredits(int delta) {
         wi.credits += delta;
         wi.adjustChannelWritability();
     }
diff --git a/hyracks-fullstack/hyracks/hyracks-net/src/test/java/org/apache/hyracks/net/tests/NetTest.java b/hyracks-fullstack/hyracks/hyracks-net/src/test/java/org/apache/hyracks/net/tests/NetTest.java
index f1f3c4a..bf01fdb 100644
--- a/hyracks-fullstack/hyracks/hyracks-net/src/test/java/org/apache/hyracks/net/tests/NetTest.java
+++ b/hyracks-fullstack/hyracks/hyracks-net/src/test/java/org/apache/hyracks/net/tests/NetTest.java
@@ -28,9 +28,8 @@
 import junit.framework.Assert;
 
 import org.junit.Test;
-
-import org.apache.hyracks.net.buffers.IBufferAcceptor;
-import org.apache.hyracks.net.buffers.ICloseableBufferAcceptor;
+import org.apache.hyracks.api.comm.IBufferAcceptor;
+import org.apache.hyracks.api.comm.ICloseableBufferAcceptor;
 import org.apache.hyracks.net.protocols.muxdemux.ChannelControlBlock;
 import org.apache.hyracks.net.protocols.muxdemux.IChannelOpenListener;
 import org.apache.hyracks.net.protocols.muxdemux.MultiplexedConnection;

-- 
To view, visit https://asterix-gerrit.ics.uci.edu/897
To unsubscribe, visit https://asterix-gerrit.ics.uci.edu/settings

Gerrit-MessageType: newchange
Gerrit-Change-Id: I5c0bd7c11c1e78954ebceff49cb274d8073a64bd
Gerrit-PatchSet: 1
Gerrit-Project: asterixdb
Gerrit-Branch: master
Gerrit-Owner: Murtadha Hubail <hu...@gmail.com>

Change in asterixdb[master]: Introduce MessagingNetworkManager for NC2NC AppMessaging

Posted by "Jenkins (Code Review)" <do...@asterixdb.incubator.apache.org>.
Jenkins has posted comments on this change.

Change subject: Introduce MessagingNetworkManager for NC2NC AppMessaging
......................................................................


Patch Set 9:

Build Started https://asterix-jenkins.ics.uci.edu/job/asterix-gerrit-notopic/2404/

-- 
To view, visit https://asterix-gerrit.ics.uci.edu/897
To unsubscribe, visit https://asterix-gerrit.ics.uci.edu/settings

Gerrit-MessageType: comment
Gerrit-Change-Id: I5c0bd7c11c1e78954ebceff49cb274d8073a64bd
Gerrit-PatchSet: 9
Gerrit-Project: asterixdb
Gerrit-Branch: master
Gerrit-Owner: Murtadha Hubail <hu...@gmail.com>
Gerrit-Reviewer: Jenkins <je...@fulliautomatix.ics.uci.edu>
Gerrit-Reviewer: Murtadha Hubail <hu...@gmail.com>
Gerrit-Reviewer: Till Westmann <ti...@apache.org>
Gerrit-Reviewer: Yingyi Bu <bu...@gmail.com>
Gerrit-Reviewer: abdullah alamoudi <ba...@gmail.com>
Gerrit-HasComments: No

Change in asterixdb[master]: Introduce MessagingNetworkManager for NC2NC AppMessaging

Posted by "Murtadha Hubail (Code Review)" <do...@asterixdb.incubator.apache.org>.
Hello Till Westmann, Jenkins,

I'd like you to reexamine a change.  Please visit

    https://asterix-gerrit.ics.uci.edu/897

to look at the new patch set (#9).

Change subject: Introduce MessagingNetworkManager for NC2NC AppMessaging
......................................................................

Introduce MessagingNetworkManager for NC2NC AppMessaging

This change introduces MessagingNetworkManager to NodeControllerService.
The MessagingNetworkManager is used to open channels that are used for NC2NC
application messaging. The read and write interfaces of the messaging channels
are set by a MessagingChannelInterfaceFactory which is set by the application.

Change-Id: I5c0bd7c11c1e78954ebceff49cb274d8073a64bd
---
M asterixdb/asterix-active/src/main/java/org/apache/asterix/active/ActiveManager.java
M asterixdb/asterix-app/src/main/java/org/apache/asterix/api/common/AsterixHyracksIntegrationUtil.java
M asterixdb/asterix-app/src/main/java/org/apache/asterix/app/nc/AsterixNCAppRuntimeContext.java
M asterixdb/asterix-app/src/main/java/org/apache/asterix/hyracks/bootstrap/NCApplicationEntryPoint.java
A asterixdb/asterix-app/src/main/java/org/apache/asterix/messaging/MessagingChannelInterfaceFactory.java
A asterixdb/asterix-app/src/main/java/org/apache/asterix/messaging/MessagingChannelReadInterface.java
A asterixdb/asterix-app/src/main/java/org/apache/asterix/messaging/MessagingChannelWriteInterface.java
M asterixdb/asterix-app/src/main/java/org/apache/asterix/messaging/NCMessageBroker.java
M asterixdb/asterix-app/src/main/resources/asterix-build-configuration.xml
M asterixdb/asterix-common/pom.xml
M asterixdb/asterix-common/src/main/java/org/apache/asterix/common/config/IAsterixPropertiesProvider.java
A asterixdb/asterix-common/src/main/java/org/apache/asterix/common/config/MessagingProperties.java
R asterixdb/asterix-common/src/main/java/org/apache/asterix/common/memory/ConcurrentFramePool.java
R asterixdb/asterix-common/src/main/java/org/apache/asterix/common/memory/FrameAction.java
M asterixdb/asterix-common/src/main/java/org/apache/asterix/common/messaging/api/IApplicationMessage.java
M asterixdb/asterix-common/src/main/java/org/apache/asterix/common/messaging/api/INCMessageBroker.java
R asterixdb/asterix-common/src/test/java/org/apache/asterix/test/memory/ConcurrentFramePoolUnitTest.java
M asterixdb/asterix-external-data/src/main/java/org/apache/asterix/external/feed/dataflow/FeedRuntimeInputHandler.java
M asterixdb/asterix-external-data/src/test/java/org/apache/asterix/external/feed/test/InputHandlerTest.java
M asterixdb/asterix-om/src/main/java/org/apache/asterix/om/util/AsterixAppContextInfo.java
M asterixdb/asterix-transactions/src/main/java/org/apache/asterix/transaction/management/resource/GlobalResourceIdFactory.java
M hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/application/INCApplicationContext.java
M hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/client/NodeControllerInfo.java
R hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/IBufferAcceptor.java
R hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/IBufferFactory.java
A hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/IChannelControlBlock.java
C hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/IChannelInterfaceFactory.java
R hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/IChannelReadInterface.java
R hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/IChannelWriteInterface.java
R hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/ICloseableBufferAcceptor.java
C hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/IConnectionWriterState.java
R hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/MuxDemuxCommand.java
R hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/exceptions/NetException.java
M hyracks-fullstack/hyracks/hyracks-client/src/main/java/org/apache/hyracks/client/net/ClientNetworkManager.java
M hyracks-fullstack/hyracks/hyracks-comm/src/main/java/org/apache/hyracks/comm/channels/DatasetNetworkInputChannel.java
M hyracks-fullstack/hyracks/hyracks-comm/src/main/java/org/apache/hyracks/comm/channels/IChannelConnectionFactory.java
M hyracks-fullstack/hyracks/hyracks-comm/src/main/java/org/apache/hyracks/comm/channels/NetworkInputChannel.java
M hyracks-fullstack/hyracks/hyracks-comm/src/main/java/org/apache/hyracks/comm/channels/NetworkOutputChannel.java
M hyracks-fullstack/hyracks/hyracks-comm/src/main/java/org/apache/hyracks/comm/channels/ReadBufferFactory.java
M hyracks-fullstack/hyracks/hyracks-control/hyracks-control-cc/src/main/java/org/apache/hyracks/control/cc/NodeControllerState.java
M hyracks-fullstack/hyracks/hyracks-control/hyracks-control-cc/src/main/java/org/apache/hyracks/control/cc/work/GetNodeControllersInfoWork.java
M hyracks-fullstack/hyracks/hyracks-control/hyracks-control-common/src/main/java/org/apache/hyracks/control/common/controllers/NCConfig.java
M hyracks-fullstack/hyracks/hyracks-control/hyracks-control-common/src/main/java/org/apache/hyracks/control/common/controllers/NodeRegistration.java
M hyracks-fullstack/hyracks/hyracks-control/hyracks-control-nc/src/main/java/org/apache/hyracks/control/nc/NCDriver.java
M hyracks-fullstack/hyracks/hyracks-control/hyracks-control-nc/src/main/java/org/apache/hyracks/control/nc/NodeControllerService.java
M hyracks-fullstack/hyracks/hyracks-control/hyracks-control-nc/src/main/java/org/apache/hyracks/control/nc/application/NCApplicationContext.java
M hyracks-fullstack/hyracks/hyracks-control/hyracks-control-nc/src/main/java/org/apache/hyracks/control/nc/net/DatasetNetworkManager.java
A hyracks-fullstack/hyracks/hyracks-control/hyracks-control-nc/src/main/java/org/apache/hyracks/control/nc/net/MessagingNetworkManager.java
M hyracks-fullstack/hyracks/hyracks-control/hyracks-control-nc/src/main/java/org/apache/hyracks/control/nc/net/NetworkManager.java
M hyracks-fullstack/hyracks/hyracks-hdfs/hyracks-hdfs-core/src/test/java/org/apache/hyracks/hdfs/scheduler/SchedulerTest.java
M hyracks-fullstack/hyracks/hyracks-hdfs/hyracks-hdfs-core/src/test/java/org/apache/hyracks/hdfs/utils/TestUtils.java
M hyracks-fullstack/hyracks/hyracks-hdfs/hyracks-hdfs-core/src/test/java/org/apache/hyracks/hdfs2/scheduler/SchedulerTest.java
M hyracks-fullstack/hyracks/hyracks-net/pom.xml
A hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/protocols/muxdemux/AbstractChannelReadInterface.java
A hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/protocols/muxdemux/AbstractChannelWriteInterface.java
M hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/protocols/muxdemux/ChannelControlBlock.java
M hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/protocols/muxdemux/ChannelSet.java
A hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/protocols/muxdemux/FullFrameChannelInterfaceFactory.java
A hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/protocols/muxdemux/FullFrameChannelReadInterface.java
A hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/protocols/muxdemux/FullFrameChannelWriteInterface.java
M hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/protocols/muxdemux/MultiplexedConnection.java
M hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/protocols/muxdemux/MuxDemux.java
M hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/protocols/tcp/ITCPConnectionEventListener.java
M hyracks-fullstack/hyracks/hyracks-net/src/test/java/org/apache/hyracks/net/tests/NetTest.java
M hyracks-fullstack/hyracks/hyracks-test-support/src/main/java/org/apache/hyracks/test/support/TestNCApplicationContext.java
65 files changed, 1,885 insertions(+), 669 deletions(-)


  git pull ssh://asterix-gerrit.ics.uci.edu:29418/asterixdb refs/changes/97/897/9
-- 
To view, visit https://asterix-gerrit.ics.uci.edu/897
To unsubscribe, visit https://asterix-gerrit.ics.uci.edu/settings

Gerrit-MessageType: newpatchset
Gerrit-Change-Id: I5c0bd7c11c1e78954ebceff49cb274d8073a64bd
Gerrit-PatchSet: 9
Gerrit-Project: asterixdb
Gerrit-Branch: master
Gerrit-Owner: Murtadha Hubail <hu...@gmail.com>
Gerrit-Reviewer: Jenkins <je...@fulliautomatix.ics.uci.edu>
Gerrit-Reviewer: Murtadha Hubail <hu...@gmail.com>
Gerrit-Reviewer: Till Westmann <ti...@apache.org>
Gerrit-Reviewer: Yingyi Bu <bu...@gmail.com>
Gerrit-Reviewer: abdullah alamoudi <ba...@gmail.com>

Change in asterixdb[master]: Introduce MessagingNetworkManager for NC2NC AppMessaging

Posted by "Jenkins (Code Review)" <do...@asterixdb.incubator.apache.org>.
Jenkins has posted comments on this change.

Change subject: Introduce MessagingNetworkManager for NC2NC AppMessaging
......................................................................


Patch Set 6:

Build Started https://asterix-jenkins.ics.uci.edu/job/asterix-gerrit-notopic/1588/

-- 
To view, visit https://asterix-gerrit.ics.uci.edu/897
To unsubscribe, visit https://asterix-gerrit.ics.uci.edu/settings

Gerrit-MessageType: comment
Gerrit-Change-Id: I5c0bd7c11c1e78954ebceff49cb274d8073a64bd
Gerrit-PatchSet: 6
Gerrit-Project: asterixdb
Gerrit-Branch: master
Gerrit-Owner: Murtadha Hubail <hu...@gmail.com>
Gerrit-Reviewer: Jenkins <je...@fulliautomatix.ics.uci.edu>
Gerrit-Reviewer: Till Westmann <ti...@apache.org>
Gerrit-Reviewer: Yingyi Bu <bu...@gmail.com>
Gerrit-Reviewer: abdullah alamoudi <ba...@gmail.com>
Gerrit-HasComments: No

Change in asterixdb[master]: Introduce MessagingNetworkManager for NC2NC AppMessaging

Posted by "Jenkins (Code Review)" <do...@asterixdb.incubator.apache.org>.
Jenkins has posted comments on this change.

Change subject: Introduce MessagingNetworkManager for NC2NC AppMessaging
......................................................................


Patch Set 7:

Integration Tests Aborted

https://asterix-jenkins.ics.uci.edu/job/asterix-gerrit-integration-tests/19/ : ABORTED

-- 
To view, visit https://asterix-gerrit.ics.uci.edu/897
To unsubscribe, visit https://asterix-gerrit.ics.uci.edu/settings

Gerrit-MessageType: comment
Gerrit-Change-Id: I5c0bd7c11c1e78954ebceff49cb274d8073a64bd
Gerrit-PatchSet: 7
Gerrit-Project: asterixdb
Gerrit-Branch: master
Gerrit-Owner: Murtadha Hubail <hu...@gmail.com>
Gerrit-Reviewer: Jenkins <je...@fulliautomatix.ics.uci.edu>
Gerrit-Reviewer: Murtadha Hubail <hu...@gmail.com>
Gerrit-Reviewer: Till Westmann <ti...@apache.org>
Gerrit-Reviewer: Yingyi Bu <bu...@gmail.com>
Gerrit-Reviewer: abdullah alamoudi <ba...@gmail.com>
Gerrit-HasComments: No

Change in asterixdb[master]: Introduce MessagingNetworkManager for NC2NC AppMessaging

Posted by "Murtadha Hubail (Code Review)" <do...@asterixdb.incubator.apache.org>.
Murtadha Hubail has posted comments on this change.

Change subject: Introduce MessagingNetworkManager for NC2NC AppMessaging
......................................................................


Patch Set 8: Code-Review+2

Comments added and the long lines with strings were split.

Forwarding Till's +2.

-- 
To view, visit https://asterix-gerrit.ics.uci.edu/897
To unsubscribe, visit https://asterix-gerrit.ics.uci.edu/settings

Gerrit-MessageType: comment
Gerrit-Change-Id: I5c0bd7c11c1e78954ebceff49cb274d8073a64bd
Gerrit-PatchSet: 8
Gerrit-Project: asterixdb
Gerrit-Branch: master
Gerrit-Owner: Murtadha Hubail <hu...@gmail.com>
Gerrit-Reviewer: Jenkins <je...@fulliautomatix.ics.uci.edu>
Gerrit-Reviewer: Murtadha Hubail <hu...@gmail.com>
Gerrit-Reviewer: Till Westmann <ti...@apache.org>
Gerrit-Reviewer: Yingyi Bu <bu...@gmail.com>
Gerrit-Reviewer: abdullah alamoudi <ba...@gmail.com>
Gerrit-HasComments: No

Change in asterixdb[master]: Introduce MessagingNetworkManager for NC2NC AppMessaging

Posted by "Jenkins (Code Review)" <do...@asterixdb.incubator.apache.org>.
Jenkins has posted comments on this change.

Change subject: Introduce MessagingNetworkManager for NC2NC AppMessaging
......................................................................


Patch Set 7:

Build Started https://asterix-jenkins.ics.uci.edu/job/asterix-gerrit-notopic/1802/

-- 
To view, visit https://asterix-gerrit.ics.uci.edu/897
To unsubscribe, visit https://asterix-gerrit.ics.uci.edu/settings

Gerrit-MessageType: comment
Gerrit-Change-Id: I5c0bd7c11c1e78954ebceff49cb274d8073a64bd
Gerrit-PatchSet: 7
Gerrit-Project: asterixdb
Gerrit-Branch: master
Gerrit-Owner: Murtadha Hubail <hu...@gmail.com>
Gerrit-Reviewer: Jenkins <je...@fulliautomatix.ics.uci.edu>
Gerrit-Reviewer: Murtadha Hubail <hu...@gmail.com>
Gerrit-Reviewer: Till Westmann <ti...@apache.org>
Gerrit-Reviewer: Yingyi Bu <bu...@gmail.com>
Gerrit-Reviewer: abdullah alamoudi <ba...@gmail.com>
Gerrit-HasComments: No

Change in asterixdb[master]: Introduce MessagingNetworkManager for NC2NC AppMessaging

Posted by "Murtadha Hubail (Code Review)" <do...@asterixdb.incubator.apache.org>.
Murtadha Hubail has posted comments on this change.

Change subject: Introduce MessagingNetworkManager for NC2NC AppMessaging
......................................................................


Patch Set 6:

(10 comments)

https://asterix-gerrit.ics.uci.edu/#/c/897/6/asterixdb/asterix-app/src/main/java/org/apache/asterix/messaging/MessagingChannelInterfaceFactory.java
File asterixdb/asterix-app/src/main/java/org/apache/asterix/messaging/MessagingChannelInterfaceFactory.java:

Line 90:                     if (LOGGER.isLoggable(Level.WARNING)) {
> Maybe this should just throw an exception? It seems that everywhere we call
Done. Since the IBufferFactory interface is in Hyracks API common project, I made it throw HyracksDataException. If you think a NetException is more appropriate, I will gladly change it.


https://asterix-gerrit.ics.uci.edu/#/c/897/6/asterixdb/asterix-app/src/main/java/org/apache/asterix/messaging/MessagingChannelReadInterface.java
File asterixdb/asterix-app/src/main/java/org/apache/asterix/messaging/MessagingChannelReadInterface.java:

Line 31: public class MessagingChannelReadInterface implements IChannelReadInterface {
> It seems that there's a lot of overlap with FullFrameChannelReadInterface. 
I introduced an abstract implementation (AbstractChannelReadInterface) and moved all common parts there.


https://asterix-gerrit.ics.uci.edu/#/c/897/6/asterixdb/asterix-app/src/main/java/org/apache/asterix/messaging/MessagingChannelWriteInterface.java
File asterixdb/asterix-app/src/main/java/org/apache/asterix/messaging/MessagingChannelWriteInterface.java:

Line 36: public class MessagingChannelWriteInterface implements IChannelWriteInterface {
> It seems that there's a lot of overlap with FullFrameChannelWriteInterface.
I introduced an abstract implementation (AbstractChannelWriteInterface) and moved all common parts there.


Line 44:     private final ICloseableBufferAcceptor fullBufferAcceptor = new ICloseableBufferAcceptor() {
> MAJOR SonarQube violation:
Moved as a private class.


https://asterix-gerrit.ics.uci.edu/#/c/897/6/asterixdb/asterix-app/src/main/java/org/apache/asterix/messaging/NCMessageBroker.java
File asterixdb/asterix-app/src/main/java/org/apache/asterix/messaging/NCMessageBroker.java:

Line 66:     private final MessageDeliveryService msgDeliverySvc;
> Do we need to keep this as a member?
No, at least not now. I moved it inside the constructor.


Line 288:                     Thread.currentThread().interrupt();
> Is there a reason why we shouldn't exit run(), if the thread gets interrupt
Please see my detailed response on this in patch set 4.


Line 291:                         LOGGER.log(Level.WARNING, "Could not process message", e);
> Could we add some info (e.g. the message id) to the warning?
Done


https://asterix-gerrit.ics.uci.edu/#/c/897/6/hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/IChannelReadInterface.java
File hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/IChannelReadInterface.java:

Line 98:     public IBufferFactory getBufferFactor();
> a/getBufferFactor/getBufferFactory/
Done


https://asterix-gerrit.ics.uci.edu/#/c/897/6/hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/exceptions/NetException.java
File hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/exceptions/NetException.java:

Line 24:     public NetException() {
> MAJOR SonarQube violation:
Done


https://asterix-gerrit.ics.uci.edu/#/c/897/6/hyracks-fullstack/hyracks/hyracks-test-support/src/main/java/org/apache/hyracks/test/support/TestNCApplicationContext.java
File hyracks-fullstack/hyracks/hyracks-test-support/src/main/java/org/apache/hyracks/test/support/TestNCApplicationContext.java:

Line 151:     public void setMessagingChannelInterfaceFactory(IChannelInterfaceFactory interfaceFactory) {
> MAJOR SonarQube violation:
Done


-- 
To view, visit https://asterix-gerrit.ics.uci.edu/897
To unsubscribe, visit https://asterix-gerrit.ics.uci.edu/settings

Gerrit-MessageType: comment
Gerrit-Change-Id: I5c0bd7c11c1e78954ebceff49cb274d8073a64bd
Gerrit-PatchSet: 6
Gerrit-Project: asterixdb
Gerrit-Branch: master
Gerrit-Owner: Murtadha Hubail <hu...@gmail.com>
Gerrit-Reviewer: Jenkins <je...@fulliautomatix.ics.uci.edu>
Gerrit-Reviewer: Murtadha Hubail <hu...@gmail.com>
Gerrit-Reviewer: Till Westmann <ti...@apache.org>
Gerrit-Reviewer: Yingyi Bu <bu...@gmail.com>
Gerrit-Reviewer: abdullah alamoudi <ba...@gmail.com>
Gerrit-HasComments: Yes

Change in asterixdb[master]: Introduce MessagingNetworkManager for NC2NC AppMessaging

Posted by "Murtadha Hubail (Code Review)" <do...@asterixdb.incubator.apache.org>.
Hello Jenkins,

I'd like you to reexamine a change.  Please visit

    https://asterix-gerrit.ics.uci.edu/897

to look at the new patch set (#3).

Change subject: Introduce MessagingNetworkManager for NC2NC AppMessaging
......................................................................

Introduce MessagingNetworkManager for NC2NC AppMessaging

This change introduces MessagingNetworkManager to NodeControllerService.
The MessagingNetworkManager is used to open channels that are used for NC2NC
application messaging. The messaging channels are passed to the application
through the IMessageBroker APIs and should be managed by the application itself.

Change-Id: I5c0bd7c11c1e78954ebceff49cb274d8073a64bd
---
M asterixdb/asterix-app/src/main/java/org/apache/asterix/api/common/AsterixHyracksIntegrationUtil.java
M asterixdb/asterix-app/src/main/java/org/apache/asterix/messaging/NCMessageBroker.java
A asterixdb/asterix-app/src/main/java/org/apache/asterix/messaging/NCMessagingHelper.java
M asterixdb/asterix-common/src/main/java/org/apache/asterix/common/messaging/api/IApplicationMessage.java
M asterixdb/asterix-common/src/main/java/org/apache/asterix/common/messaging/api/INCMessageBroker.java
M asterixdb/asterix-transactions/src/main/java/org/apache/asterix/transaction/management/resource/GlobalResourceIdFactory.java
M hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/client/NodeControllerInfo.java
R hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/IBufferAcceptor.java
R hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/IBufferFactory.java
C hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/IChannelControlBlock.java
R hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/IChannelReadInterface.java
R hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/IChannelWriteInterface.java
R hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/ICloseableBufferAcceptor.java
M hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/messages/IMessageBroker.java
M hyracks-fullstack/hyracks/hyracks-comm/src/main/java/org/apache/hyracks/comm/channels/DatasetNetworkInputChannel.java
M hyracks-fullstack/hyracks/hyracks-comm/src/main/java/org/apache/hyracks/comm/channels/NetworkInputChannel.java
M hyracks-fullstack/hyracks/hyracks-comm/src/main/java/org/apache/hyracks/comm/channels/NetworkOutputChannel.java
M hyracks-fullstack/hyracks/hyracks-comm/src/main/java/org/apache/hyracks/comm/channels/ReadBufferFactory.java
M hyracks-fullstack/hyracks/hyracks-control/hyracks-control-cc/src/main/java/org/apache/hyracks/control/cc/NodeControllerState.java
M hyracks-fullstack/hyracks/hyracks-control/hyracks-control-cc/src/main/java/org/apache/hyracks/control/cc/work/GetNodeControllersInfoWork.java
M hyracks-fullstack/hyracks/hyracks-control/hyracks-control-common/src/main/java/org/apache/hyracks/control/common/controllers/NCConfig.java
M hyracks-fullstack/hyracks/hyracks-control/hyracks-control-common/src/main/java/org/apache/hyracks/control/common/controllers/NodeRegistration.java
M hyracks-fullstack/hyracks/hyracks-control/hyracks-control-nc/src/main/java/org/apache/hyracks/control/nc/NodeControllerService.java
M hyracks-fullstack/hyracks/hyracks-control/hyracks-control-nc/src/main/java/org/apache/hyracks/control/nc/net/DatasetNetworkManager.java
A hyracks-fullstack/hyracks/hyracks-control/hyracks-control-nc/src/main/java/org/apache/hyracks/control/nc/net/MessagingNetworkManager.java
M hyracks-fullstack/hyracks/hyracks-control/hyracks-control-nc/src/main/java/org/apache/hyracks/control/nc/net/NetworkManager.java
M hyracks-fullstack/hyracks/hyracks-hdfs/hyracks-hdfs-core/src/test/java/org/apache/hyracks/hdfs/scheduler/SchedulerTest.java
M hyracks-fullstack/hyracks/hyracks-hdfs/hyracks-hdfs-core/src/test/java/org/apache/hyracks/hdfs/utils/TestUtils.java
M hyracks-fullstack/hyracks/hyracks-hdfs/hyracks-hdfs-core/src/test/java/org/apache/hyracks/hdfs2/scheduler/SchedulerTest.java
M hyracks-fullstack/hyracks/hyracks-net/pom.xml
M hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/protocols/muxdemux/ChannelControlBlock.java
M hyracks-fullstack/hyracks/hyracks-net/src/test/java/org/apache/hyracks/net/tests/NetTest.java
32 files changed, 784 insertions(+), 200 deletions(-)


  git pull ssh://asterix-gerrit.ics.uci.edu:29418/asterixdb refs/changes/97/897/3
-- 
To view, visit https://asterix-gerrit.ics.uci.edu/897
To unsubscribe, visit https://asterix-gerrit.ics.uci.edu/settings

Gerrit-MessageType: newpatchset
Gerrit-Change-Id: I5c0bd7c11c1e78954ebceff49cb274d8073a64bd
Gerrit-PatchSet: 3
Gerrit-Project: asterixdb
Gerrit-Branch: master
Gerrit-Owner: Murtadha Hubail <hu...@gmail.com>
Gerrit-Reviewer: Jenkins <je...@fulliautomatix.ics.uci.edu>

Change in asterixdb[master]: Introduce MessagingNetworkManager for NC2NC AppMessaging

Posted by "Murtadha Hubail (Code Review)" <do...@asterixdb.incubator.apache.org>.
Murtadha Hubail has posted comments on this change.

Change subject: Introduce MessagingNetworkManager for NC2NC AppMessaging
......................................................................


Patch Set 4:

(10 comments)

https://asterix-gerrit.ics.uci.edu/#/c/897/4/asterixdb/asterix-app/src/main/java/org/apache/asterix/messaging/NCMessageBroker.java
File asterixdb/asterix-app/src/main/java/org/apache/asterix/messaging/NCMessageBroker.java:

Line 292:         public void run() {
> It seems that this never stops. Would there be an issue, if we leave the lo
It depends on how NC2NC messaging will be used. If there is some kind of messaging that will be done to coordinate nodes shutdown during the JVM shutdown hook, then it is possible that this thread might get interrupted (and stopped) before these messages arrive from other nodes, causing the shutdown progress to stop forever. For example, in your WIP change (https://asterix-gerrit.ics.uci.edu/#/c/847/7), you try to properly stop the TransactionSubsystem by interrupting the checkpoint thread sleep and on catching the InterruptedException you return from the loop, and finally wait for the checkpoint thread to return on TransactionSubsystem using join(). Everything would work perfectly if the checkpoint thread was actually sleeping when it is interrupted. However, if the checkpoint thread was actually processing something and not sleeping, the join() call in the TransactionSubsystem will cause the NC shutdown to hang forever.

The proper way to stop this thread gracefully would be to pass a POSION_PILL to its queue when it is no longer needed. An even better way is use Callables to propagate exceptions that could be thrown in the graceful shutdown of the thread. Since I don't know the usage of NC2NC yet, I left it like this to be on the safe side, and this thread will be killed by the end of the shutdown hook when the process terminates.


https://asterix-gerrit.ics.uci.edu/#/c/897/4/asterixdb/asterix-app/src/main/java/org/apache/asterix/messaging/NCMessagingHelper.java
File asterixdb/asterix-app/src/main/java/org/apache/asterix/messaging/NCMessagingHelper.java:

Line 33: import org.apache.asterix.external.feed.management.ConcurrentFramePool;
> I think that we shouldn't depend on the external package here. Can we move 
I moved it to asterix-common along with its test class.


Line 50:     //TODO make these values configurable and account for their memory usage
> Yes, please.
Done


https://asterix-gerrit.ics.uci.edu/#/c/897/4/asterixdb/asterix-common/src/main/java/org/apache/asterix/common/messaging/api/INCMessageBroker.java
File asterixdb/asterix-common/src/main/java/org/apache/asterix/common/messaging/api/INCMessageBroker.java:

Line 49:     public void reportMaxResourceId() throws Exception;
> I know that it's not part of this change, but do we still need this method 
Removed.


https://asterix-gerrit.ics.uci.edu/#/c/897/4/hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/IChannelReadInterface.java
File hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/IChannelReadInterface.java:

Line 63:     public void setFlushOnCompleteRead(boolean flushOnCompleteRead);
> Would it make sense, to remove this interface and to make this a property o
Removed.


https://asterix-gerrit.ics.uci.edu/#/c/897/4/hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/IChannelWriteInterface.java
File hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/IChannelWriteInterface.java:

Line 22:  * Represents the write interface of a {@link ChannelControlBlock}.
> Update comment
Done


https://asterix-gerrit.ics.uci.edu/#/c/897/4/hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/messages/IMessageBroker.java
File hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/messages/IMessageBroker.java:

Line 27:     public default void registerMessagingChannel(String nodeId, IChannelControlBlock ccb) {
> This method looks strange on this interface. It's not clear, why we need to
Removed.


https://asterix-gerrit.ics.uci.edu/#/c/897/4/hyracks-fullstack/hyracks/hyracks-control/hyracks-control-nc/src/main/java/org/apache/hyracks/control/nc/NodeControllerService.java
File hyracks-fullstack/hyracks/hyracks-control/hyracks-control-nc/src/main/java/org/apache/hyracks/control/nc/NodeControllerService.java:

Line 256:                 ncConfig.messagingPublicPort);
> Can we make the creation (and availability) of this dependent on the applic
Done. Now applications that do not specify messaging IP or do not provide MessagingChannelInterfaceFactory will not initialize it.


https://asterix-gerrit.ics.uci.edu/#/c/897/4/hyracks-fullstack/hyracks/hyracks-control/hyracks-control-nc/src/main/java/org/apache/hyracks/control/nc/net/MessagingNetworkManager.java
File hyracks-fullstack/hyracks/hyracks-control/hyracks-control-nc/src/main/java/org/apache/hyracks/control/nc/net/MessagingNetworkManager.java:

Line 30: import org.apache.hyracks.api.exceptions.HyracksDataException;
> Could we stick with NetException in this file?
Done but I though you would suggest getting rid of the NetException completely :)


Line 48:     private final IMessageBroker messageBroker;
> It seems that this is only needed to register a new channel, but not to rou
yes, it is removed now.


-- 
To view, visit https://asterix-gerrit.ics.uci.edu/897
To unsubscribe, visit https://asterix-gerrit.ics.uci.edu/settings

Gerrit-MessageType: comment
Gerrit-Change-Id: I5c0bd7c11c1e78954ebceff49cb274d8073a64bd
Gerrit-PatchSet: 4
Gerrit-Project: asterixdb
Gerrit-Branch: master
Gerrit-Owner: Murtadha Hubail <hu...@gmail.com>
Gerrit-Reviewer: Jenkins <je...@fulliautomatix.ics.uci.edu>
Gerrit-Reviewer: Murtadha Hubail <hu...@gmail.com>
Gerrit-Reviewer: Till Westmann <ti...@apache.org>
Gerrit-Reviewer: Yingyi Bu <bu...@gmail.com>
Gerrit-Reviewer: abdullah alamoudi <ba...@gmail.com>
Gerrit-HasComments: Yes

Change in asterixdb[master]: Introduce MessagingNetworkManager for NC2NC AppMessaging

Posted by "Jenkins (Code Review)" <do...@asterixdb.incubator.apache.org>.
Jenkins has posted comments on this change.

Change subject: Introduce MessagingNetworkManager for NC2NC AppMessaging
......................................................................


Patch Set 8:

Integration Tests Started https://asterix-jenkins.ics.uci.edu/job/asterix-gerrit-integration-tests/468/

-- 
To view, visit https://asterix-gerrit.ics.uci.edu/897
To unsubscribe, visit https://asterix-gerrit.ics.uci.edu/settings

Gerrit-MessageType: comment
Gerrit-Change-Id: I5c0bd7c11c1e78954ebceff49cb274d8073a64bd
Gerrit-PatchSet: 8
Gerrit-Project: asterixdb
Gerrit-Branch: master
Gerrit-Owner: Murtadha Hubail <hu...@gmail.com>
Gerrit-Reviewer: Jenkins <je...@fulliautomatix.ics.uci.edu>
Gerrit-Reviewer: Murtadha Hubail <hu...@gmail.com>
Gerrit-Reviewer: Till Westmann <ti...@apache.org>
Gerrit-Reviewer: Yingyi Bu <bu...@gmail.com>
Gerrit-Reviewer: abdullah alamoudi <ba...@gmail.com>
Gerrit-HasComments: No

Change in asterixdb[master]: Introduce MessagingNetworkManager for NC2NC AppMessaging

Posted by "Murtadha Hubail (Code Review)" <do...@asterixdb.incubator.apache.org>.
Murtadha Hubail has posted comments on this change.

Change subject: Introduce MessagingNetworkManager for NC2NC AppMessaging
......................................................................


Patch Set 9: Code-Review+2

Forwarding Till's +2.

-- 
To view, visit https://asterix-gerrit.ics.uci.edu/897
To unsubscribe, visit https://asterix-gerrit.ics.uci.edu/settings

Gerrit-MessageType: comment
Gerrit-Change-Id: I5c0bd7c11c1e78954ebceff49cb274d8073a64bd
Gerrit-PatchSet: 9
Gerrit-Project: asterixdb
Gerrit-Branch: master
Gerrit-Owner: Murtadha Hubail <hu...@gmail.com>
Gerrit-Reviewer: Jenkins <je...@fulliautomatix.ics.uci.edu>
Gerrit-Reviewer: Murtadha Hubail <hu...@gmail.com>
Gerrit-Reviewer: Till Westmann <ti...@apache.org>
Gerrit-Reviewer: Yingyi Bu <bu...@gmail.com>
Gerrit-Reviewer: abdullah alamoudi <ba...@gmail.com>
Gerrit-HasComments: No

Change in asterixdb[master]: Introduce MessagingNetworkManager for NC2NC AppMessaging

Posted by "Till Westmann (Code Review)" <do...@asterixdb.incubator.apache.org>.
Till Westmann has posted comments on this change.

Change subject: Introduce MessagingNetworkManager for NC2NC AppMessaging
......................................................................


Patch Set 4:

(10 comments)

https://asterix-gerrit.ics.uci.edu/#/c/897/4/asterixdb/asterix-app/src/main/java/org/apache/asterix/messaging/NCMessageBroker.java
File asterixdb/asterix-app/src/main/java/org/apache/asterix/messaging/NCMessageBroker.java:

Line 292:         public void run() {
It seems that this never stops. Would there be an issue, if we leave the loop on an InterruptedException?


https://asterix-gerrit.ics.uci.edu/#/c/897/4/asterixdb/asterix-app/src/main/java/org/apache/asterix/messaging/NCMessagingHelper.java
File asterixdb/asterix-app/src/main/java/org/apache/asterix/messaging/NCMessagingHelper.java:

Line 33: import org.apache.asterix.external.feed.management.ConcurrentFramePool;
I think that we shouldn't depend on the external package here. Can we move those classes to a better (more general) place?


Line 50:     //TODO make these values configurable and account for their memory usage
Yes, please.


https://asterix-gerrit.ics.uci.edu/#/c/897/4/asterixdb/asterix-common/src/main/java/org/apache/asterix/common/messaging/api/INCMessageBroker.java
File asterixdb/asterix-common/src/main/java/org/apache/asterix/common/messaging/api/INCMessageBroker.java:

Line 49:     public void reportMaxResourceId() throws Exception;
I know that it's not part of this change, but do we still need this method on the interface. Isn't this just one special message that we sent to the CC?


https://asterix-gerrit.ics.uci.edu/#/c/897/4/hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/IChannelReadInterface.java
File hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/IChannelReadInterface.java:

Line 63:     public void setFlushOnCompleteRead(boolean flushOnCompleteRead);
Would it make sense, to remove this interface and to make this a property of the channel that is passed in on construction? It seems that this property should no change during the lifetime of a channel.


https://asterix-gerrit.ics.uci.edu/#/c/897/4/hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/IChannelWriteInterface.java
File hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/IChannelWriteInterface.java:

Line 22:  * Represents the write interface of a {@link ChannelControlBlock}.
Update comment


https://asterix-gerrit.ics.uci.edu/#/c/897/4/hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/messages/IMessageBroker.java
File hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/messages/IMessageBroker.java:

Line 27:     public default void registerMessagingChannel(String nodeId, IChannelControlBlock ccb) {
This method looks strange on this interface. It's not clear, why we need to register one kind of channels, but not the other one. Can we remove this or move it to another (parallel) interface or ...?


https://asterix-gerrit.ics.uci.edu/#/c/897/4/hyracks-fullstack/hyracks/hyracks-control/hyracks-control-nc/src/main/java/org/apache/hyracks/control/nc/NodeControllerService.java
File hyracks-fullstack/hyracks/hyracks-control/hyracks-control-nc/src/main/java/org/apache/hyracks/control/nc/NodeControllerService.java:

Line 256:                 ncConfig.messagingPublicPort);
Can we make the creation (and availability) of this dependent on the application. I think that not every Hyracks application will need (or want) to have this service.


https://asterix-gerrit.ics.uci.edu/#/c/897/4/hyracks-fullstack/hyracks/hyracks-control/hyracks-control-nc/src/main/java/org/apache/hyracks/control/nc/net/MessagingNetworkManager.java
File hyracks-fullstack/hyracks/hyracks-control/hyracks-control-nc/src/main/java/org/apache/hyracks/control/nc/net/MessagingNetworkManager.java:

Line 30: import org.apache.hyracks.api.exceptions.HyracksDataException;
Could we stick with NetException in this file?


Line 48:     private final IMessageBroker messageBroker;
It seems that this is only needed to register a new channel, but not to route an messages. If so, it doesn't need to be a message broker and the IMessageBroker interface wouldn't need the registration API. 
Is that right?


-- 
To view, visit https://asterix-gerrit.ics.uci.edu/897
To unsubscribe, visit https://asterix-gerrit.ics.uci.edu/settings

Gerrit-MessageType: comment
Gerrit-Change-Id: I5c0bd7c11c1e78954ebceff49cb274d8073a64bd
Gerrit-PatchSet: 4
Gerrit-Project: asterixdb
Gerrit-Branch: master
Gerrit-Owner: Murtadha Hubail <hu...@gmail.com>
Gerrit-Reviewer: Jenkins <je...@fulliautomatix.ics.uci.edu>
Gerrit-Reviewer: Till Westmann <ti...@apache.org>
Gerrit-Reviewer: Yingyi Bu <bu...@gmail.com>
Gerrit-Reviewer: abdullah alamoudi <ba...@gmail.com>
Gerrit-HasComments: Yes

Change in asterixdb[master]: Introduce MessagingNetworkManager for NC2NC AppMessaging

Posted by "Murtadha Hubail (Code Review)" <do...@asterixdb.incubator.apache.org>.
Hello Jenkins,

I'd like you to reexamine a change.  Please visit

    https://asterix-gerrit.ics.uci.edu/897

to look at the new patch set (#7).

Change subject: Introduce MessagingNetworkManager for NC2NC AppMessaging
......................................................................

Introduce MessagingNetworkManager for NC2NC AppMessaging

This change introduces MessagingNetworkManager to NodeControllerService.
The MessagingNetworkManager is used to open channels that are used for NC2NC
application messaging. The read and write interfaces of the messaging channels
are set by a MessagingChannelInterfaceFactory which is set by the application.

Change-Id: I5c0bd7c11c1e78954ebceff49cb274d8073a64bd
---
M asterixdb/asterix-app/src/main/java/org/apache/asterix/api/common/AsterixAppRuntimeContext.java
M asterixdb/asterix-app/src/main/java/org/apache/asterix/api/common/AsterixHyracksIntegrationUtil.java
M asterixdb/asterix-app/src/main/java/org/apache/asterix/hyracks/bootstrap/NCApplicationEntryPoint.java
A asterixdb/asterix-app/src/main/java/org/apache/asterix/messaging/MessagingChannelInterfaceFactory.java
A asterixdb/asterix-app/src/main/java/org/apache/asterix/messaging/MessagingChannelReadInterface.java
A asterixdb/asterix-app/src/main/java/org/apache/asterix/messaging/MessagingChannelWriteInterface.java
M asterixdb/asterix-app/src/main/java/org/apache/asterix/messaging/NCMessageBroker.java
M asterixdb/asterix-app/src/main/resources/asterix-build-configuration.xml
M asterixdb/asterix-common/src/main/java/org/apache/asterix/common/config/IAsterixPropertiesProvider.java
A asterixdb/asterix-common/src/main/java/org/apache/asterix/common/config/MessagingProperties.java
R asterixdb/asterix-common/src/main/java/org/apache/asterix/common/memory/ConcurrentFramePool.java
R asterixdb/asterix-common/src/main/java/org/apache/asterix/common/memory/FrameAction.java
M asterixdb/asterix-common/src/main/java/org/apache/asterix/common/messaging/api/IApplicationMessage.java
M asterixdb/asterix-common/src/main/java/org/apache/asterix/common/messaging/api/INCMessageBroker.java
R asterixdb/asterix-common/src/test/java/org/apache/asterix/test/memory/ConcurrentFramePoolUnitTest.java
M asterixdb/asterix-external-data/src/main/java/org/apache/asterix/external/feed/dataflow/FeedRuntimeInputHandler.java
M asterixdb/asterix-external-data/src/main/java/org/apache/asterix/external/feed/management/FeedManager.java
M asterixdb/asterix-external-data/src/test/java/org/apache/asterix/external/feed/test/InputHandlerTest.java
M asterixdb/asterix-om/src/main/java/org/apache/asterix/om/util/AsterixAppContextInfo.java
M asterixdb/asterix-transactions/src/main/java/org/apache/asterix/transaction/management/resource/GlobalResourceIdFactory.java
M hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/application/INCApplicationContext.java
M hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/client/NodeControllerInfo.java
R hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/IBufferAcceptor.java
R hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/IBufferFactory.java
A hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/IChannelControlBlock.java
C hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/IChannelInterfaceFactory.java
R hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/IChannelReadInterface.java
R hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/IChannelWriteInterface.java
R hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/ICloseableBufferAcceptor.java
C hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/IConnectionWriterState.java
R hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/MuxDemuxCommand.java
R hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/exceptions/NetException.java
M hyracks-fullstack/hyracks/hyracks-client/src/main/java/org/apache/hyracks/client/net/ClientNetworkManager.java
M hyracks-fullstack/hyracks/hyracks-comm/src/main/java/org/apache/hyracks/comm/channels/DatasetNetworkInputChannel.java
M hyracks-fullstack/hyracks/hyracks-comm/src/main/java/org/apache/hyracks/comm/channels/IChannelConnectionFactory.java
M hyracks-fullstack/hyracks/hyracks-comm/src/main/java/org/apache/hyracks/comm/channels/NetworkInputChannel.java
M hyracks-fullstack/hyracks/hyracks-comm/src/main/java/org/apache/hyracks/comm/channels/NetworkOutputChannel.java
M hyracks-fullstack/hyracks/hyracks-comm/src/main/java/org/apache/hyracks/comm/channels/ReadBufferFactory.java
M hyracks-fullstack/hyracks/hyracks-control/hyracks-control-cc/src/main/java/org/apache/hyracks/control/cc/NodeControllerState.java
M hyracks-fullstack/hyracks/hyracks-control/hyracks-control-cc/src/main/java/org/apache/hyracks/control/cc/work/GetNodeControllersInfoWork.java
M hyracks-fullstack/hyracks/hyracks-control/hyracks-control-common/src/main/java/org/apache/hyracks/control/common/controllers/NCConfig.java
M hyracks-fullstack/hyracks/hyracks-control/hyracks-control-common/src/main/java/org/apache/hyracks/control/common/controllers/NodeRegistration.java
M hyracks-fullstack/hyracks/hyracks-control/hyracks-control-nc/src/main/java/org/apache/hyracks/control/nc/NodeControllerService.java
M hyracks-fullstack/hyracks/hyracks-control/hyracks-control-nc/src/main/java/org/apache/hyracks/control/nc/application/NCApplicationContext.java
M hyracks-fullstack/hyracks/hyracks-control/hyracks-control-nc/src/main/java/org/apache/hyracks/control/nc/net/DatasetNetworkManager.java
A hyracks-fullstack/hyracks/hyracks-control/hyracks-control-nc/src/main/java/org/apache/hyracks/control/nc/net/MessagingNetworkManager.java
M hyracks-fullstack/hyracks/hyracks-control/hyracks-control-nc/src/main/java/org/apache/hyracks/control/nc/net/NetworkManager.java
M hyracks-fullstack/hyracks/hyracks-hdfs/hyracks-hdfs-core/src/test/java/org/apache/hyracks/hdfs/scheduler/SchedulerTest.java
M hyracks-fullstack/hyracks/hyracks-hdfs/hyracks-hdfs-core/src/test/java/org/apache/hyracks/hdfs/utils/TestUtils.java
M hyracks-fullstack/hyracks/hyracks-hdfs/hyracks-hdfs-core/src/test/java/org/apache/hyracks/hdfs2/scheduler/SchedulerTest.java
M hyracks-fullstack/hyracks/hyracks-net/pom.xml
A hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/protocols/muxdemux/AbstractChannelReadInterface.java
A hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/protocols/muxdemux/AbstractChannelWriteInterface.java
M hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/protocols/muxdemux/ChannelControlBlock.java
M hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/protocols/muxdemux/ChannelSet.java
A hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/protocols/muxdemux/FullFrameChannelInterfaceFactory.java
A hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/protocols/muxdemux/FullFrameChannelReadInterface.java
A hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/protocols/muxdemux/FullFrameChannelWriteInterface.java
M hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/protocols/muxdemux/MultiplexedConnection.java
M hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/protocols/muxdemux/MuxDemux.java
M hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/protocols/tcp/ITCPConnectionEventListener.java
M hyracks-fullstack/hyracks/hyracks-net/src/test/java/org/apache/hyracks/net/tests/NetTest.java
M hyracks-fullstack/hyracks/hyracks-test-support/src/main/java/org/apache/hyracks/test/support/TestNCApplicationContext.java
63 files changed, 1,796 insertions(+), 598 deletions(-)


  git pull ssh://asterix-gerrit.ics.uci.edu:29418/asterixdb refs/changes/97/897/7
-- 
To view, visit https://asterix-gerrit.ics.uci.edu/897
To unsubscribe, visit https://asterix-gerrit.ics.uci.edu/settings

Gerrit-MessageType: newpatchset
Gerrit-Change-Id: I5c0bd7c11c1e78954ebceff49cb274d8073a64bd
Gerrit-PatchSet: 7
Gerrit-Project: asterixdb
Gerrit-Branch: master
Gerrit-Owner: Murtadha Hubail <hu...@gmail.com>
Gerrit-Reviewer: Jenkins <je...@fulliautomatix.ics.uci.edu>
Gerrit-Reviewer: Murtadha Hubail <hu...@gmail.com>
Gerrit-Reviewer: Till Westmann <ti...@apache.org>
Gerrit-Reviewer: Yingyi Bu <bu...@gmail.com>
Gerrit-Reviewer: abdullah alamoudi <ba...@gmail.com>

Change in asterixdb[master]: Introduce MessagingNetworkManager for NC2NC AppMessaging

Posted by "Till Westmann (Code Review)" <do...@asterixdb.incubator.apache.org>.
Till Westmann has posted comments on this change.

Change subject: Introduce MessagingNetworkManager for NC2NC AppMessaging
......................................................................


Patch Set 7: Code-Review+2

Before you rebase and submit, could you

1) add a sanitized version (e.g. no links to example merge proposals) of your comments in patch set 4 on stopping the MessageDeliveryService in the the NCMessageBroker to the class and

2) fix the long lines that SonarQube is complaining about?

-- 
To view, visit https://asterix-gerrit.ics.uci.edu/897
To unsubscribe, visit https://asterix-gerrit.ics.uci.edu/settings

Gerrit-MessageType: comment
Gerrit-Change-Id: I5c0bd7c11c1e78954ebceff49cb274d8073a64bd
Gerrit-PatchSet: 7
Gerrit-Project: asterixdb
Gerrit-Branch: master
Gerrit-Owner: Murtadha Hubail <hu...@gmail.com>
Gerrit-Reviewer: Jenkins <je...@fulliautomatix.ics.uci.edu>
Gerrit-Reviewer: Murtadha Hubail <hu...@gmail.com>
Gerrit-Reviewer: Till Westmann <ti...@apache.org>
Gerrit-Reviewer: Yingyi Bu <bu...@gmail.com>
Gerrit-Reviewer: abdullah alamoudi <ba...@gmail.com>
Gerrit-HasComments: No

Change in asterixdb[master]: Introduce MessagingNetworkManager for NC2NC AppMessaging

Posted by "Jenkins (Code Review)" <do...@asterixdb.incubator.apache.org>.
Jenkins has posted comments on this change.

Change subject: Introduce MessagingNetworkManager for NC2NC AppMessaging
......................................................................


Patch Set 2:

Build Started https://asterix-jenkins.ics.uci.edu/job/asterix-gerrit-notopic/1528/

-- 
To view, visit https://asterix-gerrit.ics.uci.edu/897
To unsubscribe, visit https://asterix-gerrit.ics.uci.edu/settings

Gerrit-MessageType: comment
Gerrit-Change-Id: I5c0bd7c11c1e78954ebceff49cb274d8073a64bd
Gerrit-PatchSet: 2
Gerrit-Project: asterixdb
Gerrit-Branch: master
Gerrit-Owner: Murtadha Hubail <hu...@gmail.com>
Gerrit-Reviewer: Jenkins <je...@fulliautomatix.ics.uci.edu>
Gerrit-HasComments: No

Change in asterixdb[master]: Introduce MessagingNetworkManager for NC2NC AppMessaging

Posted by "Jenkins (Code Review)" <do...@asterixdb.incubator.apache.org>.
Jenkins has posted comments on this change.

Change subject: Introduce MessagingNetworkManager for NC2NC AppMessaging
......................................................................


Patch Set 9:

WARNING: THIS CHANGE CONTAINS CROSS-PRODUCT CHANGES IN:
* asterixdb
* hyracks-fullstack

PLEASE REVIEW CAREFULLY AND LOOK FOR API CHANGES!

-- 
To view, visit https://asterix-gerrit.ics.uci.edu/897
To unsubscribe, visit https://asterix-gerrit.ics.uci.edu/settings

Gerrit-MessageType: comment
Gerrit-Change-Id: I5c0bd7c11c1e78954ebceff49cb274d8073a64bd
Gerrit-PatchSet: 9
Gerrit-Project: asterixdb
Gerrit-Branch: master
Gerrit-Owner: Murtadha Hubail <hu...@gmail.com>
Gerrit-Reviewer: Jenkins <je...@fulliautomatix.ics.uci.edu>
Gerrit-Reviewer: Murtadha Hubail <hu...@gmail.com>
Gerrit-Reviewer: Till Westmann <ti...@apache.org>
Gerrit-Reviewer: Yingyi Bu <bu...@gmail.com>
Gerrit-Reviewer: abdullah alamoudi <ba...@gmail.com>
Gerrit-HasComments: No

Change in asterixdb[master]: Introduce MessagingNetworkManager for NC2NC AppMessaging

Posted by "Jenkins (Code Review)" <do...@asterixdb.incubator.apache.org>.
Jenkins has posted comments on this change.

Change subject: Introduce MessagingNetworkManager for NC2NC AppMessaging
......................................................................


Patch Set 8: Integration-Tests+1

Integration Tests Successful

https://asterix-jenkins.ics.uci.edu/job/asterix-gerrit-integration-tests/468/ : SUCCESS

-- 
To view, visit https://asterix-gerrit.ics.uci.edu/897
To unsubscribe, visit https://asterix-gerrit.ics.uci.edu/settings

Gerrit-MessageType: comment
Gerrit-Change-Id: I5c0bd7c11c1e78954ebceff49cb274d8073a64bd
Gerrit-PatchSet: 8
Gerrit-Project: asterixdb
Gerrit-Branch: master
Gerrit-Owner: Murtadha Hubail <hu...@gmail.com>
Gerrit-Reviewer: Jenkins <je...@fulliautomatix.ics.uci.edu>
Gerrit-Reviewer: Murtadha Hubail <hu...@gmail.com>
Gerrit-Reviewer: Till Westmann <ti...@apache.org>
Gerrit-Reviewer: Yingyi Bu <bu...@gmail.com>
Gerrit-Reviewer: abdullah alamoudi <ba...@gmail.com>
Gerrit-HasComments: No

Change in asterixdb[master]: Introduce MessagingNetworkManager for NC2NC AppMessaging

Posted by "Jenkins (Code Review)" <do...@asterixdb.incubator.apache.org>.
Jenkins has posted comments on this change.

Change subject: Introduce MessagingNetworkManager for NC2NC AppMessaging
......................................................................


Patch Set 9: Integration-Tests+1

Integration Tests Successful

https://asterix-jenkins.ics.uci.edu/job/asterix-gerrit-integration-tests/470/ : SUCCESS

-- 
To view, visit https://asterix-gerrit.ics.uci.edu/897
To unsubscribe, visit https://asterix-gerrit.ics.uci.edu/settings

Gerrit-MessageType: comment
Gerrit-Change-Id: I5c0bd7c11c1e78954ebceff49cb274d8073a64bd
Gerrit-PatchSet: 9
Gerrit-Project: asterixdb
Gerrit-Branch: master
Gerrit-Owner: Murtadha Hubail <hu...@gmail.com>
Gerrit-Reviewer: Jenkins <je...@fulliautomatix.ics.uci.edu>
Gerrit-Reviewer: Murtadha Hubail <hu...@gmail.com>
Gerrit-Reviewer: Till Westmann <ti...@apache.org>
Gerrit-Reviewer: Yingyi Bu <bu...@gmail.com>
Gerrit-Reviewer: abdullah alamoudi <ba...@gmail.com>
Gerrit-HasComments: No

Change in asterixdb[master]: Introduce MessagingNetworkManager for NC2NC AppMessaging

Posted by "Jenkins (Code Review)" <do...@asterixdb.incubator.apache.org>.
Jenkins has posted comments on this change.

Change subject: Introduce MessagingNetworkManager for NC2NC AppMessaging
......................................................................


Patch Set 8:

Build Started https://asterix-jenkins.ics.uci.edu/job/asterix-gerrit-notopic/2397/

-- 
To view, visit https://asterix-gerrit.ics.uci.edu/897
To unsubscribe, visit https://asterix-gerrit.ics.uci.edu/settings

Gerrit-MessageType: comment
Gerrit-Change-Id: I5c0bd7c11c1e78954ebceff49cb274d8073a64bd
Gerrit-PatchSet: 8
Gerrit-Project: asterixdb
Gerrit-Branch: master
Gerrit-Owner: Murtadha Hubail <hu...@gmail.com>
Gerrit-Reviewer: Jenkins <je...@fulliautomatix.ics.uci.edu>
Gerrit-Reviewer: Murtadha Hubail <hu...@gmail.com>
Gerrit-Reviewer: Till Westmann <ti...@apache.org>
Gerrit-Reviewer: Yingyi Bu <bu...@gmail.com>
Gerrit-Reviewer: abdullah alamoudi <ba...@gmail.com>
Gerrit-HasComments: No

Change in asterixdb[master]: Introduce MessagingNetworkManager for NC2NC AppMessaging

Posted by "Murtadha Hubail (Code Review)" <do...@asterixdb.incubator.apache.org>.
Hello Jenkins,

I'd like you to reexamine a change.  Please visit

    https://asterix-gerrit.ics.uci.edu/897

to look at the new patch set (#4).

Change subject: Introduce MessagingNetworkManager for NC2NC AppMessaging
......................................................................

Introduce MessagingNetworkManager for NC2NC AppMessaging

This change introduces MessagingNetworkManager to NodeControllerService.
The MessagingNetworkManager is used to open channels that are used for NC2NC
application messaging. The messaging channels are passed to the application
through the IMessageBroker APIs and should be managed by the application itself.

Change-Id: I5c0bd7c11c1e78954ebceff49cb274d8073a64bd
---
M asterixdb/asterix-app/src/main/java/org/apache/asterix/api/common/AsterixHyracksIntegrationUtil.java
M asterixdb/asterix-app/src/main/java/org/apache/asterix/messaging/NCMessageBroker.java
A asterixdb/asterix-app/src/main/java/org/apache/asterix/messaging/NCMessagingHelper.java
M asterixdb/asterix-common/src/main/java/org/apache/asterix/common/messaging/api/IApplicationMessage.java
M asterixdb/asterix-common/src/main/java/org/apache/asterix/common/messaging/api/INCMessageBroker.java
M asterixdb/asterix-transactions/src/main/java/org/apache/asterix/transaction/management/resource/GlobalResourceIdFactory.java
M hyracks-fullstack/algebricks/algebricks-tests/src/test/java/org/apache/hyracks/algebricks/tests/util/AlgebricksHyracksIntegrationUtil.java
M hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/client/NodeControllerInfo.java
R hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/IBufferAcceptor.java
R hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/IBufferFactory.java
C hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/IChannelControlBlock.java
R hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/IChannelReadInterface.java
R hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/IChannelWriteInterface.java
R hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/ICloseableBufferAcceptor.java
M hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/messages/IMessageBroker.java
M hyracks-fullstack/hyracks/hyracks-client/src/test/java/org/apache/hyracks/client/stats/HyracksUtils.java
M hyracks-fullstack/hyracks/hyracks-comm/src/main/java/org/apache/hyracks/comm/channels/DatasetNetworkInputChannel.java
M hyracks-fullstack/hyracks/hyracks-comm/src/main/java/org/apache/hyracks/comm/channels/NetworkInputChannel.java
M hyracks-fullstack/hyracks/hyracks-comm/src/main/java/org/apache/hyracks/comm/channels/NetworkOutputChannel.java
M hyracks-fullstack/hyracks/hyracks-comm/src/main/java/org/apache/hyracks/comm/channels/ReadBufferFactory.java
M hyracks-fullstack/hyracks/hyracks-control/hyracks-control-cc/src/main/java/org/apache/hyracks/control/cc/NodeControllerState.java
M hyracks-fullstack/hyracks/hyracks-control/hyracks-control-cc/src/main/java/org/apache/hyracks/control/cc/work/GetNodeControllersInfoWork.java
M hyracks-fullstack/hyracks/hyracks-control/hyracks-control-common/src/main/java/org/apache/hyracks/control/common/controllers/NCConfig.java
M hyracks-fullstack/hyracks/hyracks-control/hyracks-control-common/src/main/java/org/apache/hyracks/control/common/controllers/NodeRegistration.java
M hyracks-fullstack/hyracks/hyracks-control/hyracks-control-nc/src/main/java/org/apache/hyracks/control/nc/NodeControllerService.java
M hyracks-fullstack/hyracks/hyracks-control/hyracks-control-nc/src/main/java/org/apache/hyracks/control/nc/net/DatasetNetworkManager.java
A hyracks-fullstack/hyracks/hyracks-control/hyracks-control-nc/src/main/java/org/apache/hyracks/control/nc/net/MessagingNetworkManager.java
M hyracks-fullstack/hyracks/hyracks-control/hyracks-control-nc/src/main/java/org/apache/hyracks/control/nc/net/NetworkManager.java
M hyracks-fullstack/hyracks/hyracks-examples/hyracks-integration-tests/src/test/java/org/apache/hyracks/tests/integration/AbstractIntegrationTest.java
M hyracks-fullstack/hyracks/hyracks-examples/hyracks-integration-tests/src/test/java/org/apache/hyracks/tests/integration/AbstractMultiNCIntegrationTest.java
M hyracks-fullstack/hyracks/hyracks-hdfs/hyracks-hdfs-core/src/test/java/org/apache/hyracks/hdfs/scheduler/SchedulerTest.java
M hyracks-fullstack/hyracks/hyracks-hdfs/hyracks-hdfs-core/src/test/java/org/apache/hyracks/hdfs/utils/HyracksUtils.java
M hyracks-fullstack/hyracks/hyracks-hdfs/hyracks-hdfs-core/src/test/java/org/apache/hyracks/hdfs/utils/TestUtils.java
M hyracks-fullstack/hyracks/hyracks-hdfs/hyracks-hdfs-core/src/test/java/org/apache/hyracks/hdfs2/scheduler/SchedulerTest.java
M hyracks-fullstack/hyracks/hyracks-net/pom.xml
M hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/protocols/muxdemux/ChannelControlBlock.java
M hyracks-fullstack/hyracks/hyracks-net/src/test/java/org/apache/hyracks/net/tests/NetTest.java
37 files changed, 808 insertions(+), 214 deletions(-)


  git pull ssh://asterix-gerrit.ics.uci.edu:29418/asterixdb refs/changes/97/897/4
-- 
To view, visit https://asterix-gerrit.ics.uci.edu/897
To unsubscribe, visit https://asterix-gerrit.ics.uci.edu/settings

Gerrit-MessageType: newpatchset
Gerrit-Change-Id: I5c0bd7c11c1e78954ebceff49cb274d8073a64bd
Gerrit-PatchSet: 4
Gerrit-Project: asterixdb
Gerrit-Branch: master
Gerrit-Owner: Murtadha Hubail <hu...@gmail.com>
Gerrit-Reviewer: Jenkins <je...@fulliautomatix.ics.uci.edu>

Change in asterixdb[master]: Introduce MessagingNetworkManager for NC2NC AppMessaging

Posted by "Jenkins (Code Review)" <do...@asterixdb.incubator.apache.org>.
Jenkins has posted comments on this change.

Change subject: Introduce MessagingNetworkManager for NC2NC AppMessaging
......................................................................


Patch Set 4:

Build Started https://asterix-jenkins.ics.uci.edu/job/asterix-gerrit-notopic/1530/

-- 
To view, visit https://asterix-gerrit.ics.uci.edu/897
To unsubscribe, visit https://asterix-gerrit.ics.uci.edu/settings

Gerrit-MessageType: comment
Gerrit-Change-Id: I5c0bd7c11c1e78954ebceff49cb274d8073a64bd
Gerrit-PatchSet: 4
Gerrit-Project: asterixdb
Gerrit-Branch: master
Gerrit-Owner: Murtadha Hubail <hu...@gmail.com>
Gerrit-Reviewer: Jenkins <je...@fulliautomatix.ics.uci.edu>
Gerrit-HasComments: No

Change in asterixdb[master]: Introduce MessagingNetworkManager for NC2NC AppMessaging

Posted by "Jenkins (Code Review)" <do...@asterixdb.incubator.apache.org>.
Jenkins has posted comments on this change.

Change subject: Introduce MessagingNetworkManager for NC2NC AppMessaging
......................................................................


Patch Set 5:

Build Started https://asterix-jenkins.ics.uci.edu/job/asterix-gerrit-notopic/1586/

-- 
To view, visit https://asterix-gerrit.ics.uci.edu/897
To unsubscribe, visit https://asterix-gerrit.ics.uci.edu/settings

Gerrit-MessageType: comment
Gerrit-Change-Id: I5c0bd7c11c1e78954ebceff49cb274d8073a64bd
Gerrit-PatchSet: 5
Gerrit-Project: asterixdb
Gerrit-Branch: master
Gerrit-Owner: Murtadha Hubail <hu...@gmail.com>
Gerrit-Reviewer: Jenkins <je...@fulliautomatix.ics.uci.edu>
Gerrit-Reviewer: Till Westmann <ti...@apache.org>
Gerrit-Reviewer: Yingyi Bu <bu...@gmail.com>
Gerrit-Reviewer: abdullah alamoudi <ba...@gmail.com>
Gerrit-HasComments: No

Change in asterixdb[master]: Introduce MessagingNetworkManager for NC2NC AppMessaging

Posted by "Murtadha Hubail (Code Review)" <do...@asterixdb.incubator.apache.org>.
Hello Jenkins,

I'd like you to reexamine a change.  Please visit

    https://asterix-gerrit.ics.uci.edu/897

to look at the new patch set (#5).

Change subject: Introduce MessagingNetworkManager for NC2NC AppMessaging
......................................................................

Introduce MessagingNetworkManager for NC2NC AppMessaging

This change introduces MessagingNetworkManager to NodeControllerService.
The MessagingNetworkManager is used to open channels that are used for NC2NC
application messaging. The read and write interfaces of the messaging channels
are set by a MessagingChannelInterfaceFactory which is set by the application.

Change-Id: I5c0bd7c11c1e78954ebceff49cb274d8073a64bd
---
M asterixdb/asterix-app/src/main/java/org/apache/asterix/api/common/AsterixAppRuntimeContext.java
M asterixdb/asterix-app/src/main/java/org/apache/asterix/api/common/AsterixHyracksIntegrationUtil.java
M asterixdb/asterix-app/src/main/java/org/apache/asterix/hyracks/bootstrap/NCApplicationEntryPoint.java
A asterixdb/asterix-app/src/main/java/org/apache/asterix/messaging/MessagingChannelInterfaceFactory.java
A asterixdb/asterix-app/src/main/java/org/apache/asterix/messaging/MessagingChannelReadInterface.java
A asterixdb/asterix-app/src/main/java/org/apache/asterix/messaging/MessagingChannelWriteInterface.java
M asterixdb/asterix-app/src/main/java/org/apache/asterix/messaging/NCMessageBroker.java
M asterixdb/asterix-app/src/main/resources/asterix-build-configuration.xml
M asterixdb/asterix-common/src/main/java/org/apache/asterix/common/config/IAsterixPropertiesProvider.java
A asterixdb/asterix-common/src/main/java/org/apache/asterix/common/config/MessagingProperties.java
R asterixdb/asterix-common/src/main/java/org/apache/asterix/common/memory/ConcurrentFramePool.java
R asterixdb/asterix-common/src/main/java/org/apache/asterix/common/memory/FrameAction.java
M asterixdb/asterix-common/src/main/java/org/apache/asterix/common/messaging/api/IApplicationMessage.java
M asterixdb/asterix-common/src/main/java/org/apache/asterix/common/messaging/api/INCMessageBroker.java
R asterixdb/asterix-common/src/test/java/org/apache/asterix/test/memory/ConcurrentFramePoolUnitTest.java
M asterixdb/asterix-external-data/src/main/java/org/apache/asterix/external/feed/dataflow/FeedRuntimeInputHandler.java
M asterixdb/asterix-external-data/src/main/java/org/apache/asterix/external/feed/management/FeedManager.java
M asterixdb/asterix-external-data/src/test/java/org/apache/asterix/external/feed/test/InputHandlerTest.java
M asterixdb/asterix-om/src/main/java/org/apache/asterix/om/util/AsterixAppContextInfo.java
M asterixdb/asterix-transactions/src/main/java/org/apache/asterix/transaction/management/resource/GlobalResourceIdFactory.java
M hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/application/INCApplicationContext.java
M hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/client/NodeControllerInfo.java
R hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/IBufferAcceptor.java
R hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/IBufferFactory.java
A hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/IChannelControlBlock.java
C hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/IChannelInterfaceFactory.java
R hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/IChannelReadInterface.java
R hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/IChannelWriteInterface.java
R hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/ICloseableBufferAcceptor.java
C hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/IConnectionWriterState.java
R hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/MuxDemuxCommand.java
R hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/exceptions/NetException.java
M hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/messages/IMessageBroker.java
M hyracks-fullstack/hyracks/hyracks-client/src/main/java/org/apache/hyracks/client/net/ClientNetworkManager.java
M hyracks-fullstack/hyracks/hyracks-comm/src/main/java/org/apache/hyracks/comm/channels/DatasetNetworkInputChannel.java
M hyracks-fullstack/hyracks/hyracks-comm/src/main/java/org/apache/hyracks/comm/channels/IChannelConnectionFactory.java
M hyracks-fullstack/hyracks/hyracks-comm/src/main/java/org/apache/hyracks/comm/channels/NetworkInputChannel.java
M hyracks-fullstack/hyracks/hyracks-comm/src/main/java/org/apache/hyracks/comm/channels/NetworkOutputChannel.java
M hyracks-fullstack/hyracks/hyracks-comm/src/main/java/org/apache/hyracks/comm/channels/ReadBufferFactory.java
M hyracks-fullstack/hyracks/hyracks-control/hyracks-control-cc/src/main/java/org/apache/hyracks/control/cc/NodeControllerState.java
M hyracks-fullstack/hyracks/hyracks-control/hyracks-control-cc/src/main/java/org/apache/hyracks/control/cc/work/GetNodeControllersInfoWork.java
M hyracks-fullstack/hyracks/hyracks-control/hyracks-control-common/src/main/java/org/apache/hyracks/control/common/controllers/NCConfig.java
M hyracks-fullstack/hyracks/hyracks-control/hyracks-control-common/src/main/java/org/apache/hyracks/control/common/controllers/NodeRegistration.java
M hyracks-fullstack/hyracks/hyracks-control/hyracks-control-nc/src/main/java/org/apache/hyracks/control/nc/NodeControllerService.java
M hyracks-fullstack/hyracks/hyracks-control/hyracks-control-nc/src/main/java/org/apache/hyracks/control/nc/application/NCApplicationContext.java
M hyracks-fullstack/hyracks/hyracks-control/hyracks-control-nc/src/main/java/org/apache/hyracks/control/nc/net/DatasetNetworkManager.java
A hyracks-fullstack/hyracks/hyracks-control/hyracks-control-nc/src/main/java/org/apache/hyracks/control/nc/net/MessagingNetworkManager.java
M hyracks-fullstack/hyracks/hyracks-control/hyracks-control-nc/src/main/java/org/apache/hyracks/control/nc/net/NetworkManager.java
M hyracks-fullstack/hyracks/hyracks-hdfs/hyracks-hdfs-core/src/test/java/org/apache/hyracks/hdfs/scheduler/SchedulerTest.java
M hyracks-fullstack/hyracks/hyracks-hdfs/hyracks-hdfs-core/src/test/java/org/apache/hyracks/hdfs/utils/HyracksUtils.java
M hyracks-fullstack/hyracks/hyracks-hdfs/hyracks-hdfs-core/src/test/java/org/apache/hyracks/hdfs/utils/TestUtils.java
M hyracks-fullstack/hyracks/hyracks-hdfs/hyracks-hdfs-core/src/test/java/org/apache/hyracks/hdfs2/scheduler/SchedulerTest.java
M hyracks-fullstack/hyracks/hyracks-net/pom.xml
M hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/protocols/muxdemux/ChannelControlBlock.java
M hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/protocols/muxdemux/ChannelSet.java
A hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/protocols/muxdemux/FullFrameChannelInterfaceFactory.java
A hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/protocols/muxdemux/FullFrameChannelReadInterface.java
A hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/protocols/muxdemux/FullFrameChannelWriteInterface.java
M hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/protocols/muxdemux/MultiplexedConnection.java
M hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/protocols/muxdemux/MuxDemux.java
M hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/protocols/tcp/ITCPConnectionEventListener.java
M hyracks-fullstack/hyracks/hyracks-net/src/test/java/org/apache/hyracks/net/tests/NetTest.java
M hyracks-fullstack/hyracks/hyracks-server/src/main/java/org/apache/hyracks/server/drivers/VirtualClusterDriver.java
M hyracks-fullstack/hyracks/hyracks-test-support/src/main/java/org/apache/hyracks/test/support/TestNCApplicationContext.java
64 files changed, 1,910 insertions(+), 584 deletions(-)


  git pull ssh://asterix-gerrit.ics.uci.edu:29418/asterixdb refs/changes/97/897/5
-- 
To view, visit https://asterix-gerrit.ics.uci.edu/897
To unsubscribe, visit https://asterix-gerrit.ics.uci.edu/settings

Gerrit-MessageType: newpatchset
Gerrit-Change-Id: I5c0bd7c11c1e78954ebceff49cb274d8073a64bd
Gerrit-PatchSet: 5
Gerrit-Project: asterixdb
Gerrit-Branch: master
Gerrit-Owner: Murtadha Hubail <hu...@gmail.com>
Gerrit-Reviewer: Jenkins <je...@fulliautomatix.ics.uci.edu>
Gerrit-Reviewer: Till Westmann <ti...@apache.org>
Gerrit-Reviewer: Yingyi Bu <bu...@gmail.com>
Gerrit-Reviewer: abdullah alamoudi <ba...@gmail.com>

Change in asterixdb[master]: Introduce MessagingNetworkManager for NC2NC AppMessaging

Posted by "Murtadha Hubail (Code Review)" <do...@asterixdb.incubator.apache.org>.
Murtadha Hubail has submitted this change and it was merged.

Change subject: Introduce MessagingNetworkManager for NC2NC AppMessaging
......................................................................


Introduce MessagingNetworkManager for NC2NC AppMessaging

This change introduces MessagingNetworkManager to NodeControllerService.
The MessagingNetworkManager is used to open channels that are used for NC2NC
application messaging. The read and write interfaces of the messaging channels
are set by a MessagingChannelInterfaceFactory which is set by the application.

Change-Id: I5c0bd7c11c1e78954ebceff49cb274d8073a64bd
Reviewed-on: https://asterix-gerrit.ics.uci.edu/897
Tested-by: Jenkins <je...@fulliautomatix.ics.uci.edu>
Integration-Tests: Jenkins <je...@fulliautomatix.ics.uci.edu>
Reviewed-by: Murtadha Hubail <hu...@gmail.com>
---
M asterixdb/asterix-active/src/main/java/org/apache/asterix/active/ActiveManager.java
M asterixdb/asterix-app/src/main/java/org/apache/asterix/api/common/AsterixHyracksIntegrationUtil.java
M asterixdb/asterix-app/src/main/java/org/apache/asterix/app/nc/AsterixNCAppRuntimeContext.java
M asterixdb/asterix-app/src/main/java/org/apache/asterix/hyracks/bootstrap/NCApplicationEntryPoint.java
A asterixdb/asterix-app/src/main/java/org/apache/asterix/messaging/MessagingChannelInterfaceFactory.java
A asterixdb/asterix-app/src/main/java/org/apache/asterix/messaging/MessagingChannelReadInterface.java
A asterixdb/asterix-app/src/main/java/org/apache/asterix/messaging/MessagingChannelWriteInterface.java
M asterixdb/asterix-app/src/main/java/org/apache/asterix/messaging/NCMessageBroker.java
M asterixdb/asterix-app/src/main/resources/asterix-build-configuration.xml
M asterixdb/asterix-common/pom.xml
M asterixdb/asterix-common/src/main/java/org/apache/asterix/common/config/IAsterixPropertiesProvider.java
A asterixdb/asterix-common/src/main/java/org/apache/asterix/common/config/MessagingProperties.java
R asterixdb/asterix-common/src/main/java/org/apache/asterix/common/memory/ConcurrentFramePool.java
R asterixdb/asterix-common/src/main/java/org/apache/asterix/common/memory/FrameAction.java
M asterixdb/asterix-common/src/main/java/org/apache/asterix/common/messaging/api/IApplicationMessage.java
M asterixdb/asterix-common/src/main/java/org/apache/asterix/common/messaging/api/INCMessageBroker.java
R asterixdb/asterix-common/src/test/java/org/apache/asterix/test/memory/ConcurrentFramePoolUnitTest.java
M asterixdb/asterix-external-data/src/main/java/org/apache/asterix/external/feed/dataflow/FeedRuntimeInputHandler.java
M asterixdb/asterix-external-data/src/test/java/org/apache/asterix/external/feed/test/InputHandlerTest.java
M asterixdb/asterix-om/src/main/java/org/apache/asterix/om/util/AsterixAppContextInfo.java
M asterixdb/asterix-transactions/src/main/java/org/apache/asterix/transaction/management/resource/GlobalResourceIdFactory.java
M hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/application/INCApplicationContext.java
M hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/client/NodeControllerInfo.java
R hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/IBufferAcceptor.java
R hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/IBufferFactory.java
A hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/IChannelControlBlock.java
C hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/IChannelInterfaceFactory.java
R hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/IChannelReadInterface.java
R hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/IChannelWriteInterface.java
R hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/ICloseableBufferAcceptor.java
C hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/IConnectionWriterState.java
R hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/MuxDemuxCommand.java
R hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/exceptions/NetException.java
M hyracks-fullstack/hyracks/hyracks-client/src/main/java/org/apache/hyracks/client/net/ClientNetworkManager.java
M hyracks-fullstack/hyracks/hyracks-comm/src/main/java/org/apache/hyracks/comm/channels/DatasetNetworkInputChannel.java
M hyracks-fullstack/hyracks/hyracks-comm/src/main/java/org/apache/hyracks/comm/channels/IChannelConnectionFactory.java
M hyracks-fullstack/hyracks/hyracks-comm/src/main/java/org/apache/hyracks/comm/channels/NetworkInputChannel.java
M hyracks-fullstack/hyracks/hyracks-comm/src/main/java/org/apache/hyracks/comm/channels/NetworkOutputChannel.java
M hyracks-fullstack/hyracks/hyracks-comm/src/main/java/org/apache/hyracks/comm/channels/ReadBufferFactory.java
M hyracks-fullstack/hyracks/hyracks-control/hyracks-control-cc/src/main/java/org/apache/hyracks/control/cc/NodeControllerState.java
M hyracks-fullstack/hyracks/hyracks-control/hyracks-control-cc/src/main/java/org/apache/hyracks/control/cc/work/GetNodeControllersInfoWork.java
M hyracks-fullstack/hyracks/hyracks-control/hyracks-control-common/src/main/java/org/apache/hyracks/control/common/controllers/NCConfig.java
M hyracks-fullstack/hyracks/hyracks-control/hyracks-control-common/src/main/java/org/apache/hyracks/control/common/controllers/NodeRegistration.java
M hyracks-fullstack/hyracks/hyracks-control/hyracks-control-nc/src/main/java/org/apache/hyracks/control/nc/NCDriver.java
M hyracks-fullstack/hyracks/hyracks-control/hyracks-control-nc/src/main/java/org/apache/hyracks/control/nc/NodeControllerService.java
M hyracks-fullstack/hyracks/hyracks-control/hyracks-control-nc/src/main/java/org/apache/hyracks/control/nc/application/NCApplicationContext.java
M hyracks-fullstack/hyracks/hyracks-control/hyracks-control-nc/src/main/java/org/apache/hyracks/control/nc/net/DatasetNetworkManager.java
A hyracks-fullstack/hyracks/hyracks-control/hyracks-control-nc/src/main/java/org/apache/hyracks/control/nc/net/MessagingNetworkManager.java
M hyracks-fullstack/hyracks/hyracks-control/hyracks-control-nc/src/main/java/org/apache/hyracks/control/nc/net/NetworkManager.java
M hyracks-fullstack/hyracks/hyracks-hdfs/hyracks-hdfs-core/src/test/java/org/apache/hyracks/hdfs/scheduler/SchedulerTest.java
M hyracks-fullstack/hyracks/hyracks-hdfs/hyracks-hdfs-core/src/test/java/org/apache/hyracks/hdfs/utils/TestUtils.java
M hyracks-fullstack/hyracks/hyracks-hdfs/hyracks-hdfs-core/src/test/java/org/apache/hyracks/hdfs2/scheduler/SchedulerTest.java
M hyracks-fullstack/hyracks/hyracks-net/pom.xml
A hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/protocols/muxdemux/AbstractChannelReadInterface.java
A hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/protocols/muxdemux/AbstractChannelWriteInterface.java
M hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/protocols/muxdemux/ChannelControlBlock.java
M hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/protocols/muxdemux/ChannelSet.java
A hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/protocols/muxdemux/FullFrameChannelInterfaceFactory.java
A hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/protocols/muxdemux/FullFrameChannelReadInterface.java
A hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/protocols/muxdemux/FullFrameChannelWriteInterface.java
M hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/protocols/muxdemux/MultiplexedConnection.java
M hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/protocols/muxdemux/MuxDemux.java
M hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/protocols/tcp/ITCPConnectionEventListener.java
M hyracks-fullstack/hyracks/hyracks-net/src/test/java/org/apache/hyracks/net/tests/NetTest.java
M hyracks-fullstack/hyracks/hyracks-test-support/src/main/java/org/apache/hyracks/test/support/TestNCApplicationContext.java
65 files changed, 1,885 insertions(+), 669 deletions(-)

Approvals:
  Murtadha Hubail: Looks good to me, approved
  Jenkins: Verified; Verified



diff --git a/asterixdb/asterix-active/src/main/java/org/apache/asterix/active/ActiveManager.java b/asterixdb/asterix-active/src/main/java/org/apache/asterix/active/ActiveManager.java
index 2f34465..bd6dae9 100644
--- a/asterixdb/asterix-active/src/main/java/org/apache/asterix/active/ActiveManager.java
+++ b/asterixdb/asterix-active/src/main/java/org/apache/asterix/active/ActiveManager.java
@@ -22,6 +22,7 @@
 import java.util.concurrent.ConcurrentHashMap;
 
 import org.apache.asterix.active.message.ActiveManagerMessage;
+import org.apache.asterix.common.memory.ConcurrentFramePool;
 import org.apache.hyracks.api.exceptions.HyracksDataException;
 import org.apache.log4j.Logger;
 
diff --git a/asterixdb/asterix-app/src/main/java/org/apache/asterix/api/common/AsterixHyracksIntegrationUtil.java b/asterixdb/asterix-app/src/main/java/org/apache/asterix/api/common/AsterixHyracksIntegrationUtil.java
index 63cb2ce..44bc3bf 100644
--- a/asterixdb/asterix-app/src/main/java/org/apache/asterix/api/common/AsterixHyracksIntegrationUtil.java
+++ b/asterixdb/asterix-app/src/main/java/org/apache/asterix/api/common/AsterixHyracksIntegrationUtil.java
@@ -109,6 +109,7 @@
         ncConfig.clusterNetIPAddress = Inet4Address.getLoopbackAddress().getHostAddress();
         ncConfig.dataIPAddress = Inet4Address.getLoopbackAddress().getHostAddress();
         ncConfig.resultIPAddress = Inet4Address.getLoopbackAddress().getHostAddress();
+        ncConfig.messagingIPAddress = Inet4Address.getLoopbackAddress().getHostAddress();
         ncConfig.nodeId = ncName;
         ncConfig.resultTTL = 30000;
         ncConfig.resultSweepThreshold = 1000;
diff --git a/asterixdb/asterix-app/src/main/java/org/apache/asterix/app/nc/AsterixNCAppRuntimeContext.java b/asterixdb/asterix-app/src/main/java/org/apache/asterix/app/nc/AsterixNCAppRuntimeContext.java
index c73ab0f..38d7da3 100644
--- a/asterixdb/asterix-app/src/main/java/org/apache/asterix/app/nc/AsterixNCAppRuntimeContext.java
+++ b/asterixdb/asterix-app/src/main/java/org/apache/asterix/app/nc/AsterixNCAppRuntimeContext.java
@@ -45,6 +45,7 @@
 import org.apache.asterix.common.config.AsterixStorageProperties;
 import org.apache.asterix.common.config.AsterixTransactionProperties;
 import org.apache.asterix.common.config.IAsterixPropertiesProvider;
+import org.apache.asterix.common.config.MessagingProperties;
 import org.apache.asterix.common.context.AsterixFileMapManager;
 import org.apache.asterix.common.context.DatasetLifecycleManager;
 import org.apache.asterix.common.exceptions.ACIDException;
@@ -113,6 +114,7 @@
     private AsterixFeedProperties feedProperties;
     private AsterixBuildProperties buildProperties;
     private AsterixReplicationProperties replicationProperties;
+    private MessagingProperties messagingProperties;
 
     private AsterixThreadExecutor threadExecutor;
     private IDatasetLifecycleManager datasetLifecycleManager;
@@ -161,6 +163,7 @@
         buildProperties = new AsterixBuildProperties(propertiesAccessor);
         replicationProperties = new AsterixReplicationProperties(propertiesAccessor,
                 AsterixClusterProperties.INSTANCE.getCluster());
+        messagingProperties = new MessagingProperties(propertiesAccessor);
         this.metadataRmiPort = metadataRmiPort;
         libraryManager = new ExternalLibraryManager();
         if (extensions != null) {
@@ -383,6 +386,11 @@
     }
 
     @Override
+    public MessagingProperties getMessagingProperties() {
+        return messagingProperties;
+    }
+
+    @Override
     public ILSMOperationTracker getLSMBTreeOperationTracker(int datasetID) {
         return datasetLifecycleManager.getOperationTracker(datasetID);
     }
diff --git a/asterixdb/asterix-app/src/main/java/org/apache/asterix/hyracks/bootstrap/NCApplicationEntryPoint.java b/asterixdb/asterix-app/src/main/java/org/apache/asterix/hyracks/bootstrap/NCApplicationEntryPoint.java
index 582ff62..a8bc48f 100644
--- a/asterixdb/asterix-app/src/main/java/org/apache/asterix/hyracks/bootstrap/NCApplicationEntryPoint.java
+++ b/asterixdb/asterix-app/src/main/java/org/apache/asterix/hyracks/bootstrap/NCApplicationEntryPoint.java
@@ -34,13 +34,14 @@
 import org.apache.asterix.common.config.AsterixMetadataProperties;
 import org.apache.asterix.common.config.AsterixTransactionProperties;
 import org.apache.asterix.common.config.IAsterixPropertiesProvider;
-import org.apache.asterix.common.messaging.api.INCMessageBroker;
+import org.apache.asterix.common.config.MessagingProperties;
 import org.apache.asterix.common.replication.IRemoteRecoveryManager;
 import org.apache.asterix.common.transactions.IRecoveryManager;
 import org.apache.asterix.common.transactions.IRecoveryManager.SystemState;
 import org.apache.asterix.common.utils.StoragePathUtil;
 import org.apache.asterix.event.schema.cluster.Cluster;
 import org.apache.asterix.event.schema.cluster.Node;
+import org.apache.asterix.messaging.MessagingChannelInterfaceFactory;
 import org.apache.asterix.messaging.NCMessageBroker;
 import org.apache.asterix.metadata.bootstrap.MetadataBootstrap;
 import org.apache.asterix.om.util.AsterixClusterProperties;
@@ -116,8 +117,14 @@
         }
         runtimeContext.initialize(initialRun);
         ncApplicationContext.setApplicationObject(runtimeContext);
-        messageBroker = new NCMessageBroker((NodeControllerService) ncAppCtx.getControllerService());
+        MessagingProperties messagingProperties = ((IAsterixPropertiesProvider) runtimeContext)
+                .getMessagingProperties();
+        messageBroker = new NCMessageBroker((NodeControllerService) ncAppCtx.getControllerService(),
+                messagingProperties);
         ncApplicationContext.setMessageBroker(messageBroker);
+        MessagingChannelInterfaceFactory interfaceFactory = new MessagingChannelInterfaceFactory(
+                (NCMessageBroker) messageBroker, messagingProperties);
+        ncApplicationContext.setMessagingChannelInterfaceFactory(interfaceFactory);
 
         boolean replicationEnabled = AsterixClusterProperties.INSTANCE.isReplicationEnabled();
         boolean autoFailover = AsterixClusterProperties.INSTANCE.isAutoFailoverEnabled();
@@ -203,7 +210,7 @@
     @Override
     public void notifyStartupComplete() throws Exception {
         //Send max resource id on this NC to the CC
-        ((INCMessageBroker) ncApplicationContext.getMessageBroker()).reportMaxResourceId();
+        ((NCMessageBroker) ncApplicationContext.getMessageBroker()).reportMaxResourceId();
 
         AsterixMetadataProperties metadataProperties = ((IAsterixPropertiesProvider) runtimeContext)
                 .getMetadataProperties();
diff --git a/asterixdb/asterix-app/src/main/java/org/apache/asterix/messaging/MessagingChannelInterfaceFactory.java b/asterixdb/asterix-app/src/main/java/org/apache/asterix/messaging/MessagingChannelInterfaceFactory.java
new file mode 100644
index 0000000..33f16da
--- /dev/null
+++ b/asterixdb/asterix-app/src/main/java/org/apache/asterix/messaging/MessagingChannelInterfaceFactory.java
@@ -0,0 +1,152 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements.  See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership.  The ASF licenses this file
+ * to you 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.apache.asterix.messaging;
+
+import java.io.IOException;
+import java.nio.ByteBuffer;
+import java.util.logging.Level;
+import java.util.logging.Logger;
+
+import org.apache.asterix.common.config.MessagingProperties;
+import org.apache.asterix.common.memory.ConcurrentFramePool;
+import org.apache.asterix.common.memory.FrameAction;
+import org.apache.asterix.common.messaging.api.IApplicationMessage;
+import org.apache.hyracks.api.comm.IBufferAcceptor;
+import org.apache.hyracks.api.comm.IBufferFactory;
+import org.apache.hyracks.api.comm.IChannelControlBlock;
+import org.apache.hyracks.api.comm.IChannelInterfaceFactory;
+import org.apache.hyracks.api.comm.IChannelReadInterface;
+import org.apache.hyracks.api.comm.IChannelWriteInterface;
+import org.apache.hyracks.api.comm.ICloseableBufferAcceptor;
+import org.apache.hyracks.api.exceptions.HyracksDataException;
+import org.apache.hyracks.api.util.JavaSerializationUtils;
+
+public class MessagingChannelInterfaceFactory implements IChannelInterfaceFactory {
+
+    private static final Logger LOGGER = Logger.getLogger(MessagingChannelInterfaceFactory.class.getName());
+
+    private final NCMessageBroker messageBroker;
+    private final ConcurrentFramePool messagingFramePool;
+    /* A single buffer factory used by all messaging channels */
+    private final IBufferFactory appMessagingBufferFactor = new AppMessagingBufferFactory();
+    private final int msgFrameSize;
+    private final int channelFrameCount;
+
+    public MessagingChannelInterfaceFactory(NCMessageBroker messageBroker, MessagingProperties messagingProperties) {
+        this.messageBroker = messageBroker;
+        messagingFramePool = messageBroker.getMessagingFramePool();
+        msgFrameSize = messagingProperties.getFrameSize();
+        channelFrameCount = messagingProperties.getFrameCount();
+    }
+
+    @Override
+    public IChannelReadInterface createReadInterface(IChannelControlBlock ccb) {
+        AppMessagingEmptyBufferAcceptor readEmptyBufferAcceptor = new AppMessagingEmptyBufferAcceptor();
+        MessagingChannelReadInterface readInterface = new MessagingChannelReadInterface(readEmptyBufferAcceptor);
+        readInterface.setBufferFactory(appMessagingBufferFactor, channelFrameCount, msgFrameSize);
+        readInterface.setFullBufferAcceptor(new AppMessagingReadFullBufferAcceptor(readEmptyBufferAcceptor));
+        return readInterface;
+    }
+
+    @Override
+    public IChannelWriteInterface createWriteInterface(IChannelControlBlock ccb) {
+        MessagingChannelWriteInterface writeInterface = new MessagingChannelWriteInterface(ccb);
+        writeInterface.setBufferFactory(appMessagingBufferFactor, channelFrameCount, msgFrameSize);
+        writeInterface.setEmptyBufferAcceptor(new AppMessagingEmptyBufferAcceptor());
+        return writeInterface;
+    }
+
+    /**
+     * A buffer factory based on {@link ConcurrentFramePool}. Used
+     * for messaging channels buffers.
+     */
+    private final class AppMessagingBufferFactory implements IBufferFactory {
+        private final FrameAction frameAction = new FrameAction();
+
+        @Override
+        public ByteBuffer createBuffer() throws HyracksDataException {
+            ByteBuffer buffer = messagingFramePool.get();
+            if (buffer == null) {
+                try {
+                    messagingFramePool.subscribe(frameAction);
+                    buffer = frameAction.retrieve();
+                } catch (InterruptedException e) {
+                    Thread.currentThread().interrupt();
+                }
+            }
+            return buffer;
+        }
+    }
+
+    /**
+     * A buffer acceptor that receives the read buffers containing messages from
+     * other nodes.
+     */
+    private class AppMessagingReadFullBufferAcceptor implements ICloseableBufferAcceptor {
+        private final IBufferAcceptor recycle;
+
+        private AppMessagingReadFullBufferAcceptor(IBufferAcceptor recycle) {
+            this.recycle = recycle;
+        }
+
+        @Override
+        public void accept(ByteBuffer buffer) {
+            try {
+                IApplicationMessage receivedMsg = (IApplicationMessage) JavaSerializationUtils
+                        .deserialize(buffer.array());
+                // Queue the received message and free the network IO thread
+                messageBroker.queueReceivedMessage(receivedMsg);
+            } catch (ClassNotFoundException | IOException e) {
+                if (LOGGER.isLoggable(Level.WARNING)) {
+                    LOGGER.log(Level.WARNING, e.getMessage(), e);
+                }
+            } finally {
+                recycle.accept(buffer);
+            }
+        }
+
+        @Override
+        public void close() {
+            // Nothing to close
+        }
+
+        @Override
+        public void error(int ecode) {
+            // Errors are handled via messages
+        }
+    }
+
+    /**
+     * Empty buffer acceptor used to return the used buffers in app messaging
+     * to the buffer pool.
+     */
+    private class AppMessagingEmptyBufferAcceptor implements IBufferAcceptor {
+
+        @Override
+        public void accept(ByteBuffer buffer) {
+            try {
+                messagingFramePool.release(buffer);
+            } catch (HyracksDataException e) {
+                if (LOGGER.isLoggable(Level.WARNING)) {
+                    LOGGER.log(Level.WARNING, e.getMessage(), e);
+                }
+            }
+        }
+    }
+}
\ No newline at end of file
diff --git a/asterixdb/asterix-app/src/main/java/org/apache/asterix/messaging/MessagingChannelReadInterface.java b/asterixdb/asterix-app/src/main/java/org/apache/asterix/messaging/MessagingChannelReadInterface.java
new file mode 100644
index 0000000..42ec795
--- /dev/null
+++ b/asterixdb/asterix-app/src/main/java/org/apache/asterix/messaging/MessagingChannelReadInterface.java
@@ -0,0 +1,64 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements.  See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership.  The ASF licenses this file
+ * to you 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.apache.asterix.messaging;
+
+import java.io.IOException;
+import java.nio.channels.SocketChannel;
+
+import org.apache.hyracks.api.comm.IBufferAcceptor;
+import org.apache.hyracks.api.exceptions.NetException;
+import org.apache.hyracks.net.protocols.muxdemux.AbstractChannelReadInterface;
+
+public class MessagingChannelReadInterface extends AbstractChannelReadInterface {
+
+    MessagingChannelReadInterface(IBufferAcceptor emptyBufferAcceptor) {
+        this.emptyBufferAcceptor = emptyBufferAcceptor;
+    }
+
+    @Override
+    public int read(SocketChannel sc, int size) throws IOException, NetException {
+        while (true) {
+            if (size <= 0) {
+                return size;
+            }
+            if (currentReadBuffer == null) {
+                currentReadBuffer = bufferFactory.createBuffer();
+            }
+            int rSize = Math.min(size, currentReadBuffer.remaining());
+            if (rSize > 0) {
+                currentReadBuffer.limit(currentReadBuffer.position() + rSize);
+                int len;
+                len = sc.read(currentReadBuffer);
+                if (len < 0) {
+                    throw new NetException("Socket Closed");
+                }
+                size -= len;
+                if (len < rSize) {
+                    return size;
+                }
+            } else {
+                return size;
+            }
+
+            if (currentReadBuffer.remaining() <= 0) {
+                flush();
+            }
+        }
+    }
+}
\ No newline at end of file
diff --git a/asterixdb/asterix-app/src/main/java/org/apache/asterix/messaging/MessagingChannelWriteInterface.java b/asterixdb/asterix-app/src/main/java/org/apache/asterix/messaging/MessagingChannelWriteInterface.java
new file mode 100644
index 0000000..43c1542
--- /dev/null
+++ b/asterixdb/asterix-app/src/main/java/org/apache/asterix/messaging/MessagingChannelWriteInterface.java
@@ -0,0 +1,72 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements.  See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership.  The ASF licenses this file
+ * to you 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.apache.asterix.messaging;
+
+import org.apache.hyracks.api.comm.IBufferFactory;
+import org.apache.hyracks.api.comm.IChannelControlBlock;
+import org.apache.hyracks.api.comm.IConnectionWriterState;
+import org.apache.hyracks.api.comm.MuxDemuxCommand;
+import org.apache.hyracks.api.exceptions.NetException;
+import org.apache.hyracks.net.protocols.muxdemux.AbstractChannelWriteInterface;
+
+public class MessagingChannelWriteInterface extends AbstractChannelWriteInterface {
+
+    MessagingChannelWriteInterface(IChannelControlBlock ccb) {
+        super(ccb);
+    }
+
+    @Override
+    public void write(IConnectionWriterState writerState) throws NetException {
+        if (currentWriteBuffer == null) {
+            currentWriteBuffer = wiFullQueue.poll();
+        }
+        if (currentWriteBuffer != null) {
+            int size = currentWriteBuffer.remaining();
+            if (size > 0) {
+                writerState.getCommand().setChannelId(channelId);
+                writerState.getCommand().setCommandType(MuxDemuxCommand.CommandType.DATA);
+                writerState.getCommand().setData(size);
+                writerState.reset(currentWriteBuffer, size, ccb);
+            } else {
+                adjustChannelWritability();
+            }
+        } else if (ecode >= 0 && !ecodeSent) {
+            writerState.getCommand().setChannelId(channelId);
+            writerState.getCommand().setCommandType(MuxDemuxCommand.CommandType.ERROR);
+            writerState.getCommand().setData(ecode);
+            writerState.reset(null, 0, null);
+            ecodeSent = true;
+            ccb.reportLocalEOS();
+            adjustChannelWritability();
+        } else if (eos && !eosSent) {
+            writerState.getCommand().setChannelId(channelId);
+            writerState.getCommand().setCommandType(MuxDemuxCommand.CommandType.CLOSE_CHANNEL);
+            writerState.getCommand().setData(0);
+            writerState.reset(null, 0, null);
+            eosSent = true;
+            ccb.reportLocalEOS();
+            adjustChannelWritability();
+        }
+    }
+
+    @Override
+    public void setBufferFactory(IBufferFactory bufferFactory, int limit, int frameSize) {
+        credits = limit * frameSize;
+    }
+}
\ No newline at end of file
diff --git a/asterixdb/asterix-app/src/main/java/org/apache/asterix/messaging/NCMessageBroker.java b/asterixdb/asterix-app/src/main/java/org/apache/asterix/messaging/NCMessageBroker.java
index 74a5ba2..c7e4ac8 100644
--- a/asterixdb/asterix-app/src/main/java/org/apache/asterix/messaging/NCMessageBroker.java
+++ b/asterixdb/asterix-app/src/main/java/org/apache/asterix/messaging/NCMessageBroker.java
@@ -18,8 +18,11 @@
  */
 package org.apache.asterix.messaging;
 
+import java.io.IOException;
+import java.nio.ByteBuffer;
 import java.util.Map;
 import java.util.concurrent.ConcurrentHashMap;
+import java.util.concurrent.LinkedBlockingQueue;
 import java.util.concurrent.atomic.AtomicLong;
 import java.util.logging.Level;
 import java.util.logging.Logger;
@@ -27,6 +30,8 @@
 import org.apache.asterix.active.ActiveManager;
 import org.apache.asterix.active.message.ActiveManagerMessage;
 import org.apache.asterix.common.api.IAsterixAppRuntimeContext;
+import org.apache.asterix.common.config.MessagingProperties;
+import org.apache.asterix.common.memory.ConcurrentFramePool;
 import org.apache.asterix.common.messaging.AbstractApplicationMessage;
 import org.apache.asterix.common.messaging.CompleteFailbackRequestMessage;
 import org.apache.asterix.common.messaging.CompleteFailbackResponseMessage;
@@ -46,40 +51,63 @@
 import org.apache.asterix.event.schema.cluster.Node;
 import org.apache.asterix.metadata.bootstrap.MetadataIndexImmutableProperties;
 import org.apache.asterix.transaction.management.resource.PersistentLocalResourceRepository;
+import org.apache.hyracks.api.comm.IChannelControlBlock;
+import org.apache.hyracks.api.exceptions.HyracksDataException;
 import org.apache.hyracks.api.messages.IMessage;
 import org.apache.hyracks.api.util.JavaSerializationUtils;
 import org.apache.hyracks.control.nc.NodeControllerService;
 
 public class NCMessageBroker implements INCMessageBroker {
-    private final static Logger LOGGER = Logger.getLogger(NCMessageBroker.class.getName());
+    private static final Logger LOGGER = Logger.getLogger(NCMessageBroker.class.getName());
 
     private final NodeControllerService ncs;
     private final AtomicLong messageId = new AtomicLong(0);
     private final Map<Long, IApplicationMessageCallback> callbacks;
     private final IAsterixAppRuntimeContext appContext;
+    private final LinkedBlockingQueue<IApplicationMessage> receivedMsgsQ;
+    private final ConcurrentFramePool messagingFramePool;
+    private final int maxMsgSize;
 
-    public NCMessageBroker(NodeControllerService ncs) {
+    public NCMessageBroker(NodeControllerService ncs, MessagingProperties messagingProperties) {
         this.ncs = ncs;
         appContext = (IAsterixAppRuntimeContext) ncs.getApplicationContext().getApplicationObject();
-        callbacks = new ConcurrentHashMap<Long, IApplicationMessageCallback>();
+        callbacks = new ConcurrentHashMap<>();
+        maxMsgSize = messagingProperties.getFrameSize();
+        int messagingMemoryBudget = messagingProperties.getFrameSize() * messagingProperties.getFrameCount();
+        messagingFramePool = new ConcurrentFramePool(ncs.getId(), messagingMemoryBudget,
+                messagingProperties.getFrameSize());
+        receivedMsgsQ = new LinkedBlockingQueue<>();
+        MessageDeliveryService msgDeliverySvc = new MessageDeliveryService();
+        appContext.getThreadExecutor().execute(msgDeliverySvc);
     }
 
     @Override
-    public void sendMessage(IApplicationMessage message, IApplicationMessageCallback callback) throws Exception {
-        if (callback != null) {
-            long uniqueMessageId = messageId.incrementAndGet();
-            message.setId(uniqueMessageId);
-            callbacks.put(uniqueMessageId, callback);
-        }
+    public void sendMessageToCC(IApplicationMessage message, IApplicationMessageCallback callback) throws Exception {
+        registerMsgCallback(message, callback);
         try {
             ncs.sendApplicationMessageToCC(JavaSerializationUtils.serialize(message), null);
         } catch (Exception e) {
-            if (callback != null) {
-                //remove the callback in case of failure
-                callbacks.remove(message.getId());
-            }
+            handleMsgDeliveryFailure(message);
             throw e;
         }
+    }
+
+    @Override
+    public void sendMessageToNC(String nodeId, IApplicationMessage message, IApplicationMessageCallback callback)
+            throws Exception {
+        registerMsgCallback(message, callback);
+        try {
+            IChannelControlBlock messagingChannel = ncs.getMessagingNetworkManager().getMessagingChannel(nodeId);
+            sendMessageToChannel(messagingChannel, message);
+        } catch (Exception e) {
+            handleMsgDeliveryFailure(message);
+            throw e;
+        }
+    }
+
+    @Override
+    public void queueReceivedMessage(IApplicationMessage msg) {
+        receivedMsgsQ.offer(msg);
     }
 
     @Override
@@ -122,9 +150,44 @@
                     break;
             }
         } catch (Exception e) {
-            e.printStackTrace();
+            if (LOGGER.isLoggable(Level.WARNING)) {
+                LOGGER.log(Level.WARNING, e.getMessage(), e);
+            }
             throw e;
         }
+    }
+
+    public ConcurrentFramePool getMessagingFramePool() {
+        return messagingFramePool;
+    }
+
+    private void registerMsgCallback(IApplicationMessage message, IApplicationMessageCallback callback) {
+        if (callback != null) {
+            long uniqueMessageId = messageId.incrementAndGet();
+            message.setId(uniqueMessageId);
+            callbacks.put(uniqueMessageId, callback);
+        }
+    }
+
+    private void handleMsgDeliveryFailure(IApplicationMessage message) {
+        callbacks.remove(message.getId());
+    }
+
+    private void sendMessageToChannel(IChannelControlBlock ccb, IApplicationMessage msg) throws IOException {
+        byte[] serializedMsg = JavaSerializationUtils.serialize(msg);
+        if (serializedMsg.length > maxMsgSize) {
+            throw new HyracksDataException("Message exceded maximum size");
+        }
+        // Prepare the message buffer
+        ByteBuffer msgBuffer = messagingFramePool.get();
+        if (msgBuffer == null) {
+            throw new HyracksDataException("Could not get an empty buffer");
+        }
+        msgBuffer.clear();
+        msgBuffer.put(serializedMsg);
+        msgBuffer.flip();
+        // Give the buffer to the channel write interface for writing
+        ccb.getWriteInterface().getFullBufferAcceptor().accept(msgBuffer);
     }
 
     private void handleTakeoverPartitons(IMessage message) throws Exception {
@@ -138,7 +201,7 @@
                 //send response after takeover is completed
                 TakeoverPartitionsResponseMessage reponse = new TakeoverPartitionsResponseMessage(msg.getRequestId(),
                         appContext.getTransactionSubsystem().getId(), msg.getPartitions());
-                sendMessage(reponse, null);
+                sendMessageToCC(reponse, null);
             }
         }
     }
@@ -148,20 +211,19 @@
             appContext.initializeMetadata(false);
             appContext.exportMetadataNodeStub();
         } finally {
-            TakeoverMetadataNodeResponseMessage reponse =
-                    new TakeoverMetadataNodeResponseMessage(appContext.getTransactionSubsystem().getId());
-            sendMessage(reponse, null);
+            TakeoverMetadataNodeResponseMessage reponse = new TakeoverMetadataNodeResponseMessage(
+                    appContext.getTransactionSubsystem().getId());
+            sendMessageToCC(reponse, null);
         }
     }
 
-    @Override
     public void reportMaxResourceId() throws Exception {
         ReportMaxResourceIdMessage maxResourceIdMsg = new ReportMaxResourceIdMessage();
         //resource ids < FIRST_AVAILABLE_USER_DATASET_ID are reserved for metadata indexes.
         long maxResourceId = Math.max(appContext.getLocalResourceRepository().getMaxResourceID(),
                 MetadataIndexImmutableProperties.FIRST_AVAILABLE_USER_DATASET_ID);
         maxResourceIdMsg.setMaxResourceId(maxResourceId);
-        sendMessage(maxResourceIdMsg, null);
+        sendMessageToCC(maxResourceIdMsg, null);
     }
 
     private void handleReplicaEvent(IMessage message) {
@@ -194,16 +256,16 @@
         }
 
         //mark the partitions to be closed as inactive
-        PersistentLocalResourceRepository localResourceRepo =
-                (PersistentLocalResourceRepository) appContext.getLocalResourceRepository();
+        PersistentLocalResourceRepository localResourceRepo = (PersistentLocalResourceRepository) appContext
+                .getLocalResourceRepository();
         for (Integer partitionId : msg.getPartitions()) {
             localResourceRepo.addInactivePartition(partitionId);
         }
 
         //send response after partitions prepared for failback
-        PreparePartitionsFailbackResponseMessage reponse =
-                new PreparePartitionsFailbackResponseMessage(msg.getPlanId(), msg.getRequestId(), msg.getPartitions());
-        sendMessage(reponse, null);
+        PreparePartitionsFailbackResponseMessage reponse = new PreparePartitionsFailbackResponseMessage(msg.getPlanId(),
+                msg.getRequestId(), msg.getPartitions());
+        sendMessageToCC(reponse, null);
     }
 
     private void handleCompleteFailbackRequest(IMessage message) throws Exception {
@@ -212,9 +274,42 @@
             IRemoteRecoveryManager remoteRecoeryManager = appContext.getRemoteRecoveryManager();
             remoteRecoeryManager.completeFailbackProcess();
         } finally {
-            CompleteFailbackResponseMessage reponse =
-                    new CompleteFailbackResponseMessage(msg.getPlanId(), msg.getRequestId(), msg.getPartitions());
-            sendMessage(reponse, null);
+            CompleteFailbackResponseMessage reponse = new CompleteFailbackResponseMessage(msg.getPlanId(),
+                    msg.getRequestId(), msg.getPartitions());
+            sendMessageToCC(reponse, null);
+        }
+    }
+
+    private class MessageDeliveryService implements Runnable {
+        /*
+         * TODO Currently this thread is not stopped when it is interrupted because
+         * NC2NC messaging might be used during nodes shutdown coordination and the
+         * JVM shutdown hook might interrupt while it is still needed. If NC2NC
+         * messaging wont be used during shutdown, then this thread needs to be
+         * gracefully stopped using a POSION_PILL or when interrupted during the
+         * shutdown.
+         */
+        @Override
+        public void run() {
+            while (true) {
+                IApplicationMessage msg = null;
+                try {
+                    msg = receivedMsgsQ.take();
+                    //TODO add nodeId to IApplicationMessage and pass it
+                    receivedMessage(msg, null);
+                } catch (InterruptedException e) {
+                    Thread.currentThread().interrupt();
+                } catch (Exception e) {
+                    if (LOGGER.isLoggable(Level.WARNING) && msg != null) {
+                        LOGGER.log(Level.WARNING, "Could not process message with id: " + msg.getId() + " and type: "
+                                + msg.getMessageType().name(), e);
+                    } else {
+                        if (LOGGER.isLoggable(Level.WARNING)) {
+                            LOGGER.log(Level.WARNING, "Could not process message", e);
+                        }
+                    }
+                }
+            }
         }
     }
 }
diff --git a/asterixdb/asterix-app/src/main/resources/asterix-build-configuration.xml b/asterixdb/asterix-app/src/main/resources/asterix-build-configuration.xml
index cbd5c93..544f4ec 100644
--- a/asterixdb/asterix-app/src/main/resources/asterix-build-configuration.xml
+++ b/asterixdb/asterix-app/src/main/resources/asterix-build-configuration.xml
@@ -98,4 +98,16 @@
     <description>Enabling plot of Algebricks plan to tmp folder. (Default = false)
     </description>
   </property>
+  <property>
+    <name>messaging.frame.size</name>
+    <value>4096</value>
+    <description>The frame size to be used for NC to NC messaging. (Default = 4kb)
+    </description>
+  </property>
+  <property>
+    <name>messaging.frame.count</name>
+    <value>512</value>
+    <description>Number of reusable frames for NC to NC messaging. (Default = 512)
+    </description>
+  </property>
 </asterixConfiguration>
diff --git a/asterixdb/asterix-common/pom.xml b/asterixdb/asterix-common/pom.xml
index ab511fb..bfdc834 100644
--- a/asterixdb/asterix-common/pom.xml
+++ b/asterixdb/asterix-common/pom.xml
@@ -257,6 +257,12 @@
       <artifactId>log4j</artifactId>
       <version>1.2.17</version>
     </dependency>
+    <dependency>
+      <groupId>org.mockito</groupId>
+      <artifactId>mockito-all</artifactId>
+      <version>2.0.2-beta</version>
+      <scope>test</scope>
+    </dependency>
   </dependencies>
 
 </project>
diff --git a/asterixdb/asterix-common/src/main/java/org/apache/asterix/common/config/IAsterixPropertiesProvider.java b/asterixdb/asterix-common/src/main/java/org/apache/asterix/common/config/IAsterixPropertiesProvider.java
index e6f383f..43425ab 100644
--- a/asterixdb/asterix-common/src/main/java/org/apache/asterix/common/config/IAsterixPropertiesProvider.java
+++ b/asterixdb/asterix-common/src/main/java/org/apache/asterix/common/config/IAsterixPropertiesProvider.java
@@ -34,4 +34,6 @@
     AsterixBuildProperties getBuildProperties();
 
     public AsterixReplicationProperties getReplicationProperties();
+
+    public MessagingProperties getMessagingProperties();
 }
diff --git a/asterixdb/asterix-common/src/main/java/org/apache/asterix/common/config/MessagingProperties.java b/asterixdb/asterix-common/src/main/java/org/apache/asterix/common/config/MessagingProperties.java
new file mode 100644
index 0000000..6794c8d
--- /dev/null
+++ b/asterixdb/asterix-common/src/main/java/org/apache/asterix/common/config/MessagingProperties.java
@@ -0,0 +1,45 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements.  See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership.  The ASF licenses this file
+ * to you 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.apache.asterix.common.config;
+
+import org.apache.hyracks.util.StorageUtil;
+import org.apache.hyracks.util.StorageUtil.StorageUnit;
+
+public class MessagingProperties extends AbstractAsterixProperties {
+
+    private static final String MESSAGING_FRAME_SIZE_KEY = "messaging.frame.size";
+    private static final int MESSAGING_FRAME_SIZE_DEFAULT = StorageUtil.getSizeInBytes(4, StorageUnit.KILOBYTE);
+
+    private static final String MESSAGING_FRAME_COUNT_KEY = "messaging.frame.count";
+    private static final int MESSAGING_BUFFER_COUNTE_DEFAULT = 512;
+
+    public MessagingProperties(AsterixPropertiesAccessor accessor) {
+        super(accessor);
+    }
+
+    public int getFrameSize() {
+        return accessor.getProperty(MESSAGING_FRAME_SIZE_KEY, MESSAGING_FRAME_SIZE_DEFAULT,
+                PropertyInterpreters.getIntegerPropertyInterpreter());
+    }
+
+    public int getFrameCount() {
+        return accessor.getProperty(MESSAGING_FRAME_COUNT_KEY, MESSAGING_BUFFER_COUNTE_DEFAULT,
+                PropertyInterpreters.getIntegerPropertyInterpreter());
+    }
+}
\ No newline at end of file
diff --git a/asterixdb/asterix-active/src/main/java/org/apache/asterix/active/ConcurrentFramePool.java b/asterixdb/asterix-common/src/main/java/org/apache/asterix/common/memory/ConcurrentFramePool.java
similarity index 94%
rename from asterixdb/asterix-active/src/main/java/org/apache/asterix/active/ConcurrentFramePool.java
rename to asterixdb/asterix-common/src/main/java/org/apache/asterix/common/memory/ConcurrentFramePool.java
index afe3b06..d57a4fc 100644
--- a/asterixdb/asterix-active/src/main/java/org/apache/asterix/active/ConcurrentFramePool.java
+++ b/asterixdb/asterix-common/src/main/java/org/apache/asterix/common/memory/ConcurrentFramePool.java
@@ -16,7 +16,7 @@
  * specific language governing permissions and limitations
  * under the License.
  */
-package org.apache.asterix.active;
+package org.apache.asterix.common.memory;
 
 import java.nio.ByteBuffer;
 import java.util.ArrayDeque;
@@ -25,16 +25,17 @@
 import java.util.Iterator;
 import java.util.Map;
 import java.util.Map.Entry;
+import java.util.logging.Level;
+import java.util.logging.Logger;
 
 import org.apache.hyracks.api.exceptions.HyracksDataException;
-import org.apache.log4j.Logger;
 
 public class ConcurrentFramePool {
     private static final boolean DEBUG = false;
-    private static final String ERROR_INVALID_FRAME_SIZE =
-            "The size should be an integral multiple of the default frame size";
-    private static final String ERROR_LARGER_THAN_BUDGET_REQUEST =
-            "The requested frame size must not be greater than the allocated budget";
+    private static final String ERROR_INVALID_FRAME_SIZE = "The size should be an integral "
+            + "multiple of the default frame size";
+    private static final String ERROR_LARGER_THAN_BUDGET_REQUEST = "The requested frame size"
+            + " must not be greater than the allocated budget";
     private static final Logger LOGGER = Logger.getLogger(ConcurrentFramePool.class.getName());
     private final String nodeId;
     private final int budget;
@@ -232,7 +233,8 @@
                 try {
                     frameAction.call(freeBuffer);
                 } catch (Exception e) {
-                    LOGGER.error("Error while attempting to answer a subscription. Buffer will be reclaimed", e);
+                    LOGGER.log(Level.SEVERE,
+                            "Error while attempting to answer a subscription. Buffer will be reclaimed", e);
                     // TODO(amoudi): Add test cases and get rid of recursion
                     if (handedOut == handedOutBeforeCall) {
                         release(freeBuffer);
diff --git a/asterixdb/asterix-active/src/main/java/org/apache/asterix/active/FrameAction.java b/asterixdb/asterix-common/src/main/java/org/apache/asterix/common/memory/FrameAction.java
similarity index 96%
rename from asterixdb/asterix-active/src/main/java/org/apache/asterix/active/FrameAction.java
rename to asterixdb/asterix-common/src/main/java/org/apache/asterix/common/memory/FrameAction.java
index 849d360..366125a 100644
--- a/asterixdb/asterix-active/src/main/java/org/apache/asterix/active/FrameAction.java
+++ b/asterixdb/asterix-common/src/main/java/org/apache/asterix/common/memory/FrameAction.java
@@ -16,11 +16,10 @@
  * specific language governing permissions and limitations
  * under the License.
  */
-package org.apache.asterix.active;
+package org.apache.asterix.common.memory;
 
 import java.nio.ByteBuffer;
-
-import org.apache.log4j.Logger;
+import java.util.logging.Logger;
 
 public class FrameAction {
     private static final boolean DEBUG = false;
diff --git a/asterixdb/asterix-common/src/main/java/org/apache/asterix/common/messaging/api/IApplicationMessage.java b/asterixdb/asterix-common/src/main/java/org/apache/asterix/common/messaging/api/IApplicationMessage.java
index 9742a6c..5f08dd8 100644
--- a/asterixdb/asterix-common/src/main/java/org/apache/asterix/common/messaging/api/IApplicationMessage.java
+++ b/asterixdb/asterix-common/src/main/java/org/apache/asterix/common/messaging/api/IApplicationMessage.java
@@ -44,7 +44,7 @@
 
     /**
      * Sets a unique message id that identifies this message within an NC.
-     * This id is set by {@link INCMessageBroker#sendMessage(IApplicationMessage, IApplicationMessageCallback)}
+     * This id is set by {@link INCMessageBroker#sendMessageToCC(IApplicationMessage, IApplicationMessageCallback)}
      * when the callback is not null to notify the sender when the response to that message is received.
      *
      * @param messageId
diff --git a/asterixdb/asterix-common/src/main/java/org/apache/asterix/common/messaging/api/INCMessageBroker.java b/asterixdb/asterix-common/src/main/java/org/apache/asterix/common/messaging/api/INCMessageBroker.java
index 41f8a0c..f01d0c3 100644
--- a/asterixdb/asterix-common/src/main/java/org/apache/asterix/common/messaging/api/INCMessageBroker.java
+++ b/asterixdb/asterix-common/src/main/java/org/apache/asterix/common/messaging/api/INCMessageBroker.java
@@ -29,12 +29,22 @@
      * @param callback
      * @throws Exception
      */
-    public void sendMessage(IApplicationMessage message, IApplicationMessageCallback callback) throws Exception;
+    public void sendMessageToCC(IApplicationMessage message, IApplicationMessageCallback callback) throws Exception;
 
     /**
-     * Sends the maximum resource id on this NC to the CC.
+     * Sends application message from this NC to another NC.
      *
+     * @param message
+     * @param callback
      * @throws Exception
      */
-    public void reportMaxResourceId() throws Exception;
+    public void sendMessageToNC(String nodeId, IApplicationMessage message, IApplicationMessageCallback callback)
+            throws Exception;
+
+    /**
+     * Queue a message to this {@link INCMessageBroker} for processing
+     *
+     * @param msg
+     */
+    public void queueReceivedMessage(IApplicationMessage msg);
 }
diff --git a/asterixdb/asterix-external-data/src/test/java/org/apache/asterix/external/feed/test/ConcurrentFramePoolUnitTest.java b/asterixdb/asterix-common/src/test/java/org/apache/asterix/test/memory/ConcurrentFramePoolUnitTest.java
similarity index 92%
rename from asterixdb/asterix-external-data/src/test/java/org/apache/asterix/external/feed/test/ConcurrentFramePoolUnitTest.java
rename to asterixdb/asterix-common/src/test/java/org/apache/asterix/test/memory/ConcurrentFramePoolUnitTest.java
index 0f6a2ea..10c6b24 100644
--- a/asterixdb/asterix-external-data/src/test/java/org/apache/asterix/external/feed/test/ConcurrentFramePoolUnitTest.java
+++ b/asterixdb/asterix-common/src/test/java/org/apache/asterix/test/memory/ConcurrentFramePoolUnitTest.java
@@ -16,7 +16,7 @@
  * specific language governing permissions and limitations
  * under the License.
  */
-package org.apache.asterix.external.feed.test;
+package org.apache.asterix.test.memory;
 
 import java.nio.ByteBuffer;
 import java.util.ArrayDeque;
@@ -24,9 +24,9 @@
 import java.util.Random;
 import java.util.concurrent.LinkedBlockingDeque;
 
-import org.apache.asterix.active.ConcurrentFramePool;
-import org.apache.asterix.active.FrameAction;
 import org.apache.asterix.common.config.AsterixFeedProperties;
+import org.apache.asterix.common.memory.ConcurrentFramePool;
+import org.apache.asterix.common.memory.FrameAction;
 import org.apache.hyracks.api.exceptions.HyracksDataException;
 import org.junit.Assert;
 import org.mockito.Mockito;
@@ -60,8 +60,8 @@
     public void testMemoryManager() {
         AsterixFeedProperties afp = Mockito.mock(AsterixFeedProperties.class);
         Mockito.when(afp.getMemoryComponentGlobalBudget()).thenReturn(FEED_MEM_BUDGET);
-        ConcurrentFramePool fmm =
-                new ConcurrentFramePool("TestNode", afp.getMemoryComponentGlobalBudget(), DEFAULT_FRAME_SIZE);
+        ConcurrentFramePool fmm = new ConcurrentFramePool("TestNode", afp.getMemoryComponentGlobalBudget(),
+                DEFAULT_FRAME_SIZE);
         int i = 0;
         while (fmm.get() != null) {
             i++;
@@ -75,8 +75,8 @@
         try {
             AsterixFeedProperties afp = Mockito.mock(AsterixFeedProperties.class);
             Mockito.when(afp.getMemoryComponentGlobalBudget()).thenReturn(FEED_MEM_BUDGET);
-            ConcurrentFramePool fmm =
-                    new ConcurrentFramePool("TestNode", afp.getMemoryComponentGlobalBudget(), DEFAULT_FRAME_SIZE);
+            ConcurrentFramePool fmm = new ConcurrentFramePool("TestNode", afp.getMemoryComponentGlobalBudget(),
+                    DEFAULT_FRAME_SIZE);
             FixedSizeAllocator[] runners = new FixedSizeAllocator[NUM_THREADS];
             Thread[] threads = new Thread[NUM_THREADS];
             Arrays.parallelSetAll(runners, (int i) -> new FixedSizeAllocator(fmm));
@@ -106,8 +106,8 @@
         try {
             AsterixFeedProperties afp = Mockito.mock(AsterixFeedProperties.class);
             Mockito.when(afp.getMemoryComponentGlobalBudget()).thenReturn(FEED_MEM_BUDGET);
-            ConcurrentFramePool fmm =
-                    new ConcurrentFramePool("TestNode", afp.getMemoryComponentGlobalBudget(), DEFAULT_FRAME_SIZE);
+            ConcurrentFramePool fmm = new ConcurrentFramePool("TestNode", afp.getMemoryComponentGlobalBudget(),
+                    DEFAULT_FRAME_SIZE);
             Random random = new Random();
             int i = 0;
             int req;
@@ -141,8 +141,8 @@
         try {
             AsterixFeedProperties afp = Mockito.mock(AsterixFeedProperties.class);
             Mockito.when(afp.getMemoryComponentGlobalBudget()).thenReturn(FEED_MEM_BUDGET);
-            ConcurrentFramePool fmm =
-                    new ConcurrentFramePool("TestNode", afp.getMemoryComponentGlobalBudget(), DEFAULT_FRAME_SIZE);
+            ConcurrentFramePool fmm = new ConcurrentFramePool("TestNode", afp.getMemoryComponentGlobalBudget(),
+                    DEFAULT_FRAME_SIZE);
 
             VarSizeAllocator[] runners = new VarSizeAllocator[NUM_THREADS];
             Thread[] threads = new Thread[NUM_THREADS];
@@ -180,8 +180,8 @@
     public void testAcquireReleaseMemoryManager() throws HyracksDataException {
         AsterixFeedProperties afp = Mockito.mock(AsterixFeedProperties.class);
         Mockito.when(afp.getMemoryComponentGlobalBudget()).thenReturn(FEED_MEM_BUDGET);
-        ConcurrentFramePool fmm =
-                new ConcurrentFramePool("TestNode", afp.getMemoryComponentGlobalBudget(), DEFAULT_FRAME_SIZE);
+        ConcurrentFramePool fmm = new ConcurrentFramePool("TestNode", afp.getMemoryComponentGlobalBudget(),
+                DEFAULT_FRAME_SIZE);
         Random random = new Random();
         ArrayDeque<ByteBuffer> stack = new ArrayDeque<>();
         while (true) {
@@ -213,8 +213,8 @@
         try {
             AsterixFeedProperties afp = Mockito.mock(AsterixFeedProperties.class);
             Mockito.when(afp.getMemoryComponentGlobalBudget()).thenReturn(FEED_MEM_BUDGET);
-            ConcurrentFramePool fmm =
-                    new ConcurrentFramePool("TestNode", afp.getMemoryComponentGlobalBudget(), DEFAULT_FRAME_SIZE);
+            ConcurrentFramePool fmm = new ConcurrentFramePool("TestNode", afp.getMemoryComponentGlobalBudget(),
+                    DEFAULT_FRAME_SIZE);
             FixedSizeGoodAllocator[] runners = new FixedSizeGoodAllocator[NUM_THREADS];
             Thread[] threads = new Thread[NUM_THREADS];
             Arrays.parallelSetAll(runners, (int i) -> new FixedSizeGoodAllocator(fmm));
@@ -244,8 +244,8 @@
         try {
             AsterixFeedProperties afp = Mockito.mock(AsterixFeedProperties.class);
             Mockito.when(afp.getMemoryComponentGlobalBudget()).thenReturn(FEED_MEM_BUDGET);
-            ConcurrentFramePool fmm =
-                    new ConcurrentFramePool("TestNode", afp.getMemoryComponentGlobalBudget(), DEFAULT_FRAME_SIZE);
+            ConcurrentFramePool fmm = new ConcurrentFramePool("TestNode", afp.getMemoryComponentGlobalBudget(),
+                    DEFAULT_FRAME_SIZE);
             Random random = new Random();
             ArrayDeque<ByteBuffer> stack = new ArrayDeque<>();
             int i = 0;
@@ -297,8 +297,8 @@
         try {
             AsterixFeedProperties afp = Mockito.mock(AsterixFeedProperties.class);
             Mockito.when(afp.getMemoryComponentGlobalBudget()).thenReturn(FEED_MEM_BUDGET);
-            ConcurrentFramePool fmm =
-                    new ConcurrentFramePool("TestNode", afp.getMemoryComponentGlobalBudget(), DEFAULT_FRAME_SIZE);
+            ConcurrentFramePool fmm = new ConcurrentFramePool("TestNode", afp.getMemoryComponentGlobalBudget(),
+                    DEFAULT_FRAME_SIZE);
             VarSizeGoodAllocator[] runners = new VarSizeGoodAllocator[NUM_THREADS];
             Thread[] threads = new Thread[NUM_THREADS];
             Arrays.parallelSetAll(runners, (int i) -> new VarSizeGoodAllocator(fmm));
@@ -333,8 +333,8 @@
         try {
             AsterixFeedProperties afp = Mockito.mock(AsterixFeedProperties.class);
             Mockito.when(afp.getMemoryComponentGlobalBudget()).thenReturn(FEED_MEM_BUDGET);
-            ConcurrentFramePool fmm =
-                    new ConcurrentFramePool("TestNode", afp.getMemoryComponentGlobalBudget(), DEFAULT_FRAME_SIZE);
+            ConcurrentFramePool fmm = new ConcurrentFramePool("TestNode", afp.getMemoryComponentGlobalBudget(),
+                    DEFAULT_FRAME_SIZE);
             int i = 0;
             ByteBuffer buffer = ByteBuffer.allocate(DEFAULT_FRAME_SIZE);
             LinkedBlockingDeque<ByteBuffer> buffers = new LinkedBlockingDeque<>();
@@ -399,8 +399,8 @@
         try {
             AsterixFeedProperties afp = Mockito.mock(AsterixFeedProperties.class);
             Mockito.when(afp.getMemoryComponentGlobalBudget()).thenReturn(FEED_MEM_BUDGET);
-            ConcurrentFramePool fmm =
-                    new ConcurrentFramePool("TestNode", afp.getMemoryComponentGlobalBudget(), DEFAULT_FRAME_SIZE);
+            ConcurrentFramePool fmm = new ConcurrentFramePool("TestNode", afp.getMemoryComponentGlobalBudget(),
+                    DEFAULT_FRAME_SIZE);
             int i = 0;
             ByteBuffer buffer = ByteBuffer.allocate(DEFAULT_FRAME_SIZE);
             LinkedBlockingDeque<ByteBuffer> buffers = new LinkedBlockingDeque<>();
diff --git a/asterixdb/asterix-external-data/src/main/java/org/apache/asterix/external/feed/dataflow/FeedRuntimeInputHandler.java b/asterixdb/asterix-external-data/src/main/java/org/apache/asterix/external/feed/dataflow/FeedRuntimeInputHandler.java
index d4e3641..cd04515 100644
--- a/asterixdb/asterix-external-data/src/main/java/org/apache/asterix/external/feed/dataflow/FeedRuntimeInputHandler.java
+++ b/asterixdb/asterix-external-data/src/main/java/org/apache/asterix/external/feed/dataflow/FeedRuntimeInputHandler.java
@@ -24,8 +24,8 @@
 import java.util.logging.Logger;
 
 import org.apache.asterix.active.ActiveRuntimeId;
-import org.apache.asterix.active.ConcurrentFramePool;
-import org.apache.asterix.active.FrameAction;
+import org.apache.asterix.common.memory.ConcurrentFramePool;
+import org.apache.asterix.common.memory.FrameAction;
 import org.apache.asterix.external.feed.management.FeedConnectionId;
 import org.apache.asterix.external.feed.policy.FeedPolicyAccessor;
 import org.apache.asterix.external.util.FeedUtils.Mode;
@@ -73,11 +73,12 @@
             throws HyracksDataException {
         this.writer = writer;
 
-        this.spiller =
-                fpa.spillToDiskOnCongestion() ? new FrameSpiller(ctx,
+        this.spiller = fpa.spillToDiskOnCongestion()
+                ? new FrameSpiller(ctx,
                         connectionId.getFeedId() + "_" + connectionId.getDatasetName() + "_"
                                 + runtimeId.getRuntimeName() + "_" + runtimeId.getPartition(),
-                        fpa.getMaxSpillOnDisk()) : null;
+                        fpa.getMaxSpillOnDisk())
+                : null;
         this.exceptionHandler = new FeedExceptionHandler(ctx, fta);
         this.fpa = fpa;
         this.framePool = framePool;
diff --git a/asterixdb/asterix-external-data/src/test/java/org/apache/asterix/external/feed/test/InputHandlerTest.java b/asterixdb/asterix-external-data/src/test/java/org/apache/asterix/external/feed/test/InputHandlerTest.java
index e643206..171d271 100644
--- a/asterixdb/asterix-external-data/src/test/java/org/apache/asterix/external/feed/test/InputHandlerTest.java
+++ b/asterixdb/asterix-external-data/src/test/java/org/apache/asterix/external/feed/test/InputHandlerTest.java
@@ -26,8 +26,8 @@
 import java.util.concurrent.Future;
 
 import org.apache.asterix.active.ActiveRuntimeId;
-import org.apache.asterix.active.ConcurrentFramePool;
 import org.apache.asterix.active.EntityId;
+import org.apache.asterix.common.memory.ConcurrentFramePool;
 import org.apache.asterix.external.feed.dataflow.FeedRuntimeInputHandler;
 import org.apache.asterix.external.feed.management.FeedConnectionId;
 import org.apache.asterix.external.feed.policy.FeedPolicyAccessor;
@@ -114,11 +114,11 @@
             Random random = new Random();
             IHyracksTaskContext ctx = TestUtils.create(DEFAULT_FRAME_SIZE);
             // No spill, No discard
-            FeedPolicyAccessor fpa =
-                    createFeedPolicyAccessor(true, false, NUM_FRAMES * DEFAULT_FRAME_SIZE, DISCARD_ALLOWANCE);
+            FeedPolicyAccessor fpa = createFeedPolicyAccessor(true, false, NUM_FRAMES * DEFAULT_FRAME_SIZE,
+                    DISCARD_ALLOWANCE);
             // Non-Active Writer
-            TestFrameWriter writer =
-                    FrameWriterTestUtils.create(Collections.emptyList(), Collections.emptyList(), false);
+            TestFrameWriter writer = FrameWriterTestUtils.create(Collections.emptyList(), Collections.emptyList(),
+                    false);
             // FramePool
             ConcurrentFramePool framePool = new ConcurrentFramePool(NODE_ID, 0, DEFAULT_FRAME_SIZE);
             FeedRuntimeInputHandler handler = createInputHandler(ctx, writer, fpa, framePool);
@@ -156,11 +156,11 @@
             int numRounds = 10;
             IHyracksTaskContext ctx = TestUtils.create(DEFAULT_FRAME_SIZE);
             // No spill, No discard
-            FeedPolicyAccessor fpa =
-                    createFeedPolicyAccessor(true, false, NUM_FRAMES * DEFAULT_FRAME_SIZE, DISCARD_ALLOWANCE);
+            FeedPolicyAccessor fpa = createFeedPolicyAccessor(true, false, NUM_FRAMES * DEFAULT_FRAME_SIZE,
+                    DISCARD_ALLOWANCE);
             // Non-Active Writer
-            TestFrameWriter writer =
-                    FrameWriterTestUtils.create(Collections.emptyList(), Collections.emptyList(), false);
+            TestFrameWriter writer = FrameWriterTestUtils.create(Collections.emptyList(), Collections.emptyList(),
+                    false);
             // FramePool
             ConcurrentFramePool framePool = new ConcurrentFramePool(NODE_ID, 0, DEFAULT_FRAME_SIZE);
             FeedRuntimeInputHandler handler = createInputHandler(ctx, writer, fpa, framePool);
@@ -205,14 +205,14 @@
             int totalMinFrames = 0;
             IHyracksTaskContext ctx = TestUtils.create(DEFAULT_FRAME_SIZE);
             // Spill budget = Memory budget, No discard
-            FeedPolicyAccessor fpa =
-                    createFeedPolicyAccessor(true, true, DEFAULT_FRAME_SIZE * numberOfSpillFrames, DISCARD_ALLOWANCE);
+            FeedPolicyAccessor fpa = createFeedPolicyAccessor(true, true, DEFAULT_FRAME_SIZE * numberOfSpillFrames,
+                    DISCARD_ALLOWANCE);
             // Non-Active Writer
             TestControlledFrameWriter writer = FrameWriterTestUtils.create(DEFAULT_FRAME_SIZE, false);
             writer.freeze();
             // FramePool
-            ConcurrentFramePool framePool =
-                    new ConcurrentFramePool(NODE_ID, numberOfMemoryFrames * DEFAULT_FRAME_SIZE, DEFAULT_FRAME_SIZE);
+            ConcurrentFramePool framePool = new ConcurrentFramePool(NODE_ID, numberOfMemoryFrames * DEFAULT_FRAME_SIZE,
+                    DEFAULT_FRAME_SIZE);
             FeedRuntimeInputHandler handler = createInputHandler(ctx, writer, fpa, framePool);
             handler.open();
             ByteBuffer buffer1 = ByteBuffer.allocate(DEFAULT_FRAME_SIZE);
@@ -275,8 +275,8 @@
             Assert.assertEquals(0, handler.getNumDiscarded());
             // We can only discard one frame
             double numDiscarded = 0;
-            boolean nextShouldDiscard =
-                    ((numDiscarded + 1.0) / (handler.getTotal() + 1.0)) <= fpa.getMaxFractionDiscard();
+            boolean nextShouldDiscard = ((numDiscarded + 1.0) / (handler.getTotal() + 1.0)) <= fpa
+                    .getMaxFractionDiscard();
             while (nextShouldDiscard) {
                 handler.nextFrame(buffer5);
                 numDiscarded++;
@@ -315,14 +315,14 @@
             int numberOfSpillFrames = 50;
             IHyracksTaskContext ctx = TestUtils.create(DEFAULT_FRAME_SIZE);
             // Spill budget = Memory budget, No discard
-            FeedPolicyAccessor fpa =
-                    createFeedPolicyAccessor(true, true, DEFAULT_FRAME_SIZE * numberOfSpillFrames, DISCARD_ALLOWANCE);
+            FeedPolicyAccessor fpa = createFeedPolicyAccessor(true, true, DEFAULT_FRAME_SIZE * numberOfSpillFrames,
+                    DISCARD_ALLOWANCE);
             // Non-Active Writer
             TestControlledFrameWriter writer = FrameWriterTestUtils.create(DEFAULT_FRAME_SIZE, false);
             writer.freeze();
             // FramePool
-            ConcurrentFramePool framePool =
-                    new ConcurrentFramePool(NODE_ID, numberOfMemoryFrames * DEFAULT_FRAME_SIZE, DEFAULT_FRAME_SIZE);
+            ConcurrentFramePool framePool = new ConcurrentFramePool(NODE_ID, numberOfMemoryFrames * DEFAULT_FRAME_SIZE,
+                    DEFAULT_FRAME_SIZE);
             FeedRuntimeInputHandler handler = createInputHandler(ctx, writer, fpa, framePool);
             handler.open();
             VSizeFrame frame = new VSizeFrame(ctx);
@@ -345,8 +345,8 @@
             Assert.assertEquals(0, handler.getNumDiscarded());
             // We can only discard one frame
             double numDiscarded = 0;
-            boolean nextShouldDiscard =
-                    ((numDiscarded + 1.0) / (handler.getTotal() + 1.0)) <= fpa.getMaxFractionDiscard();
+            boolean nextShouldDiscard = ((numDiscarded + 1.0) / (handler.getTotal() + 1.0)) <= fpa
+                    .getMaxFractionDiscard();
             while (nextShouldDiscard) {
                 handler.nextFrame(frame.getBuffer());
                 numDiscarded++;
@@ -394,8 +394,8 @@
             TestControlledFrameWriter writer = FrameWriterTestUtils.create(DEFAULT_FRAME_SIZE, false);
             writer.freeze();
             // FramePool
-            ConcurrentFramePool framePool =
-                    new ConcurrentFramePool(NODE_ID, discardTestFrames * DEFAULT_FRAME_SIZE, DEFAULT_FRAME_SIZE);
+            ConcurrentFramePool framePool = new ConcurrentFramePool(NODE_ID, discardTestFrames * DEFAULT_FRAME_SIZE,
+                    DEFAULT_FRAME_SIZE);
             FeedRuntimeInputHandler handler = createInputHandler(ctx, writer, fpa, framePool);
             handler.open();
             // add NUM_FRAMES times
@@ -411,8 +411,8 @@
             }
             // Next call should NOT block but should discard.
             double numDiscarded = 0.0;
-            boolean nextShouldDiscard =
-                    ((numDiscarded + 1.0) / (handler.getTotal() + 1.0)) <= fpa.getMaxFractionDiscard();
+            boolean nextShouldDiscard = ((numDiscarded + 1.0) / (handler.getTotal() + 1.0)) <= fpa
+                    .getMaxFractionDiscard();
             while (nextShouldDiscard) {
                 handler.nextFrame(buffer);
                 numDiscarded++;
@@ -456,8 +456,8 @@
             TestControlledFrameWriter writer = FrameWriterTestUtils.create(DEFAULT_FRAME_SIZE, false);
             writer.freeze();
             // FramePool
-            ConcurrentFramePool framePool =
-                    new ConcurrentFramePool(NODE_ID, discardTestFrames * DEFAULT_FRAME_SIZE, DEFAULT_FRAME_SIZE);
+            ConcurrentFramePool framePool = new ConcurrentFramePool(NODE_ID, discardTestFrames * DEFAULT_FRAME_SIZE,
+                    DEFAULT_FRAME_SIZE);
             FeedRuntimeInputHandler handler = createInputHandler(ctx, writer, fpa, framePool);
             handler.open();
             VSizeFrame frame = new VSizeFrame(ctx);
@@ -467,8 +467,8 @@
             }
             // Next 5 calls call should NOT block but should discard.
             double numDiscarded = 0.0;
-            boolean nextShouldDiscard =
-                    ((numDiscarded + 1.0) / (handler.getTotal() + 1.0)) <= fpa.getMaxFractionDiscard();
+            boolean nextShouldDiscard = ((numDiscarded + 1.0) / (handler.getTotal() + 1.0)) <= fpa
+                    .getMaxFractionDiscard();
             while (nextShouldDiscard) {
                 handler.nextFrame(frame.getBuffer());
                 numDiscarded++;
@@ -507,8 +507,8 @@
         try {
             IHyracksTaskContext ctx = TestUtils.create(DEFAULT_FRAME_SIZE);
             // Spill budget = Memory budget, No discard
-            FeedPolicyAccessor fpa =
-                    createFeedPolicyAccessor(true, false, DEFAULT_FRAME_SIZE * NUM_FRAMES, DISCARD_ALLOWANCE);
+            FeedPolicyAccessor fpa = createFeedPolicyAccessor(true, false, DEFAULT_FRAME_SIZE * NUM_FRAMES,
+                    DISCARD_ALLOWANCE);
             // Non-Active Writer
             TestControlledFrameWriter writer = FrameWriterTestUtils.create(DEFAULT_FRAME_SIZE, false);
             writer.freeze();
@@ -554,8 +554,8 @@
             // No spill, No discard
             FeedPolicyAccessor fpa = createFeedPolicyAccessor(false, false, 0L, DISCARD_ALLOWANCE);
             // Non-Active Writer
-            TestFrameWriter writer =
-                    FrameWriterTestUtils.create(Collections.emptyList(), Collections.emptyList(), false);
+            TestFrameWriter writer = FrameWriterTestUtils.create(Collections.emptyList(), Collections.emptyList(),
+                    false);
             // FramePool
             ConcurrentFramePool framePool = new ConcurrentFramePool(NODE_ID, FEED_MEM_BUDGET, DEFAULT_FRAME_SIZE);
             FeedRuntimeInputHandler handler = createInputHandler(ctx, writer, fpa, framePool);
@@ -595,8 +595,8 @@
             // No spill, No discard
             FeedPolicyAccessor fpa = createFeedPolicyAccessor(false, false, 0L, DISCARD_ALLOWANCE);
             // Non-Active Writer
-            TestFrameWriter writer =
-                    FrameWriterTestUtils.create(Collections.emptyList(), Collections.emptyList(), false);
+            TestFrameWriter writer = FrameWriterTestUtils.create(Collections.emptyList(), Collections.emptyList(),
+                    false);
             // FramePool
             ConcurrentFramePool framePool = new ConcurrentFramePool(NODE_ID, FEED_MEM_BUDGET, DEFAULT_FRAME_SIZE);
             FeedRuntimeInputHandler handler = createInputHandler(ctx, writer, fpa, framePool);
@@ -683,8 +683,8 @@
                 Random random = new Random();
                 IHyracksTaskContext ctx = TestUtils.create(DEFAULT_FRAME_SIZE);
                 // Spill budget = Memory budget, No discard
-                FeedPolicyAccessor fpa =
-                        createFeedPolicyAccessor(true, false, DEFAULT_FRAME_SIZE * NUM_FRAMES, DISCARD_ALLOWANCE);
+                FeedPolicyAccessor fpa = createFeedPolicyAccessor(true, false, DEFAULT_FRAME_SIZE * NUM_FRAMES,
+                        DISCARD_ALLOWANCE);
                 // Non-Active Writer
                 TestControlledFrameWriter writer = FrameWriterTestUtils.create(DEFAULT_FRAME_SIZE, false);
                 writer.freeze();
diff --git a/asterixdb/asterix-om/src/main/java/org/apache/asterix/om/util/AsterixAppContextInfo.java b/asterixdb/asterix-om/src/main/java/org/apache/asterix/om/util/AsterixAppContextInfo.java
index bf103fc..e7b15a2 100644
--- a/asterixdb/asterix-om/src/main/java/org/apache/asterix/om/util/AsterixAppContextInfo.java
+++ b/asterixdb/asterix-om/src/main/java/org/apache/asterix/om/util/AsterixAppContextInfo.java
@@ -33,6 +33,7 @@
 import org.apache.asterix.common.config.AsterixStorageProperties;
 import org.apache.asterix.common.config.AsterixTransactionProperties;
 import org.apache.asterix.common.config.IAsterixPropertiesProvider;
+import org.apache.asterix.common.config.MessagingProperties;
 import org.apache.asterix.common.dataflow.IAsterixApplicationContextInfo;
 import org.apache.asterix.common.exceptions.AsterixException;
 import org.apache.asterix.common.library.ILibraryManager;
@@ -62,6 +63,7 @@
     private AsterixBuildProperties buildProperties;
     private AsterixReplicationProperties replicationProperties;
     private AsterixExtensionProperties extensionProperties;
+    private MessagingProperties messagingProperties;
     private final IGlobalRecoveryMaanger globalRecoveryMaanger;
     private IHyracksClientConnection hcc;
     private final ILibraryManager libraryManager;
@@ -92,10 +94,11 @@
         INSTANCE.txnProperties = new AsterixTransactionProperties(propertiesAccessor);
         INSTANCE.feedProperties = new AsterixFeedProperties(propertiesAccessor);
         INSTANCE.extensionProperties = new AsterixExtensionProperties(propertiesAccessor);
-        INSTANCE.replicationProperties =
-                new AsterixReplicationProperties(propertiesAccessor, AsterixClusterProperties.INSTANCE.getCluster());
+        INSTANCE.replicationProperties = new AsterixReplicationProperties(propertiesAccessor,
+                AsterixClusterProperties.INSTANCE.getCluster());
         INSTANCE.hcc = hcc;
         INSTANCE.buildProperties = new AsterixBuildProperties(propertiesAccessor);
+        INSTANCE.messagingProperties = new MessagingProperties(propertiesAccessor);
         Logger.getLogger("org.apache").setLevel(INSTANCE.externalProperties.getLogLevel());
     }
 
@@ -191,4 +194,9 @@
     public AsterixExtensionProperties getExtensionProperties() {
         return extensionProperties;
     }
+
+    @Override
+    public MessagingProperties getMessagingProperties() {
+        return messagingProperties;
+    }
 }
diff --git a/asterixdb/asterix-transactions/src/main/java/org/apache/asterix/transaction/management/resource/GlobalResourceIdFactory.java b/asterixdb/asterix-transactions/src/main/java/org/apache/asterix/transaction/management/resource/GlobalResourceIdFactory.java
index 5b29530..ab1ebe1 100644
--- a/asterixdb/asterix-transactions/src/main/java/org/apache/asterix/transaction/management/resource/GlobalResourceIdFactory.java
+++ b/asterixdb/asterix-transactions/src/main/java/org/apache/asterix/transaction/management/resource/GlobalResourceIdFactory.java
@@ -57,7 +57,7 @@
             //if no response available or it has an exception, request a new one
             if (reponse == null || reponse.getException() != null) {
                 ResourceIdRequestMessage msg = new ResourceIdRequestMessage();
-                ((INCMessageBroker) appCtx.getMessageBroker()).sendMessage(msg, this);
+                ((INCMessageBroker) appCtx.getMessageBroker()).sendMessageToCC(msg, this);
                 reponse = (ResourceIdRequestResponseMessage) resourceIdResponseQ.take();
                 if (reponse.getException() != null) {
                     throw new HyracksDataException(reponse.getException().getMessage());
diff --git a/hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/application/INCApplicationContext.java b/hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/application/INCApplicationContext.java
index 77934c6..b1aa45f 100644
--- a/hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/application/INCApplicationContext.java
+++ b/hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/application/INCApplicationContext.java
@@ -18,6 +18,7 @@
  */
 package org.apache.hyracks.api.application;
 
+import org.apache.hyracks.api.comm.IChannelInterfaceFactory;
 import org.apache.hyracks.api.context.IHyracksRootContext;
 import org.apache.hyracks.api.lifecycle.ILifeCycleComponentManager;
 import org.apache.hyracks.api.resources.memory.IMemoryManager;
@@ -77,4 +78,19 @@
      * @param handler
      */
     public void setStateDumpHandler(IStateDumpHandler handler);
+
+    /**
+     * Set the application MessagingChannelInterfaceFactory
+     *
+     * @param interfaceFactory
+     */
+    public void setMessagingChannelInterfaceFactory(IChannelInterfaceFactory interfaceFactory);
+
+    /**
+     * Get the application MessagingChannelInterfaceFactory previously set by
+     * the {@link #setMessagingChannelInterfaceFactory(IChannelInterfaceFactory)} call.
+     *
+     * @return
+     */
+    public IChannelInterfaceFactory getMessagingChannelInterfaceFactory();
 }
diff --git a/hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/client/NodeControllerInfo.java b/hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/client/NodeControllerInfo.java
index c41dafe..a79b955 100644
--- a/hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/client/NodeControllerInfo.java
+++ b/hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/client/NodeControllerInfo.java
@@ -33,12 +33,15 @@
 
     private final NetworkAddress datasetNetworkAddress;
 
+    private final NetworkAddress messagingNetworkAddress;
+
     public NodeControllerInfo(String nodeId, NodeStatus status, NetworkAddress netAddress,
-            NetworkAddress datasetNetworkAddress) {
+            NetworkAddress datasetNetworkAddress, NetworkAddress messagingNetworkAddress) {
         this.nodeId = nodeId;
         this.status = status;
         this.netAddress = netAddress;
         this.datasetNetworkAddress = datasetNetworkAddress;
+        this.messagingNetworkAddress = messagingNetworkAddress;
     }
 
     public String getNodeId() {
@@ -56,4 +59,8 @@
     public NetworkAddress getDatasetNetworkAddress() {
         return datasetNetworkAddress;
     }
+
+    public NetworkAddress getMessagingNetworkAddress() {
+        return messagingNetworkAddress;
+    }
 }
diff --git a/hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/buffers/IBufferAcceptor.java b/hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/IBufferAcceptor.java
similarity index 94%
rename from hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/buffers/IBufferAcceptor.java
rename to hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/IBufferAcceptor.java
index d7febd2..585b6bd 100644
--- a/hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/buffers/IBufferAcceptor.java
+++ b/hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/IBufferAcceptor.java
@@ -16,7 +16,7 @@
  * specific language governing permissions and limitations
  * under the License.
  */
-package org.apache.hyracks.net.buffers;
+package org.apache.hyracks.api.comm;
 
 import java.nio.ByteBuffer;
 
@@ -25,6 +25,7 @@
  *
  * @author vinayakb
  */
+@FunctionalInterface
 public interface IBufferAcceptor {
     /**
      * Accept a buffer.
diff --git a/hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/protocols/muxdemux/IBufferFactory.java b/hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/IBufferFactory.java
similarity index 82%
rename from hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/protocols/muxdemux/IBufferFactory.java
rename to hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/IBufferFactory.java
index 53299d8..5b3a233 100644
--- a/hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/protocols/muxdemux/IBufferFactory.java
+++ b/hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/IBufferFactory.java
@@ -16,15 +16,15 @@
  * specific language governing permissions and limitations
  * under the License.
  */
-package org.apache.hyracks.net.protocols.muxdemux;
+package org.apache.hyracks.api.comm;
 
 import java.nio.ByteBuffer;
 
-/**
- * @author yingyib
- */
+import org.apache.hyracks.api.exceptions.HyracksDataException;
+
+@FunctionalInterface
 public interface IBufferFactory {
 
-    public ByteBuffer createBuffer();
+    public ByteBuffer createBuffer() throws HyracksDataException;
 
-}
+}
\ No newline at end of file
diff --git a/hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/IChannelControlBlock.java b/hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/IChannelControlBlock.java
new file mode 100644
index 0000000..02de858
--- /dev/null
+++ b/hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/IChannelControlBlock.java
@@ -0,0 +1,83 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements.  See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership.  The ASF licenses this file
+ * to you 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.apache.hyracks.api.comm;
+
+public interface IChannelControlBlock {
+
+    /**
+     * Get the read interface of this channel.
+     *
+     * @return the read interface.
+     */
+    public IChannelReadInterface getReadInterface();
+
+    /**
+     * Get the write interface of this channel.
+     *
+     * @return the write interface.
+     */
+    public IChannelWriteInterface getWriteInterface();
+
+    /**
+     * Add write credit to this channel.
+     *
+     * @param delta
+     *            number of bytes
+     */
+    public void addWriteCredits(int delta);
+
+    /**
+     * @return The channel's unique id within its ChannelSet.
+     */
+    public int getChannelId();
+
+    /**
+     * Add pending credit.
+     *
+     * @param credit
+     */
+    public void addPendingCredits(int credit);
+
+    /**
+     * Increments the pending write operations of this channel.
+     */
+    public void markPendingWrite();
+
+    /**
+     * Clears the pending write operations of this channel.
+     */
+    public void unmarkPendingWrite();
+
+    /**
+     * Sets a flag indicating this channel was closed locally.
+     */
+    public void reportLocalEOS();
+
+    /**
+     * A flag indicating if the channel was closed on the remote side.
+     *
+     * @return
+     */
+    public boolean isRemotelyClosed();
+
+    /**
+     * Complete the current write operation on this channel.
+     */
+    public void writeComplete();
+}
diff --git a/hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/buffers/ICloseableBufferAcceptor.java b/hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/IChannelInterfaceFactory.java
similarity index 60%
copy from hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/buffers/ICloseableBufferAcceptor.java
copy to hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/IChannelInterfaceFactory.java
index 1697cfd..d147fc7 100644
--- a/hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/buffers/ICloseableBufferAcceptor.java
+++ b/hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/IChannelInterfaceFactory.java
@@ -16,25 +16,25 @@
  * specific language governing permissions and limitations
  * under the License.
  */
-package org.apache.hyracks.net.buffers;
+package org.apache.hyracks.api.comm;
 
-/**
- * A buffer acceptor that can be closed to indicate end of transmission or an error code
- * specified to indicate an error in transmission.
- *
- * @author vinayakb
- */
-public interface ICloseableBufferAcceptor extends IBufferAcceptor {
-    /**
-     * Close the buffer acceptor.
-     */
-    public void close();
+public interface IChannelInterfaceFactory {
 
     /**
-     * Indicate that an error occurred.
+     * Creates {@link IChannelReadInterface} and assigns the passed
+     * {@link IChannelControlBlock} to it.
      *
-     * @param ecode
-     *            - the error code.
+     * @param ccb
+     * @return
      */
-    public void error(int ecode);
+    public IChannelReadInterface createReadInterface(IChannelControlBlock ccb);
+
+    /**
+     * Creates {@link IChannelWriteInterface} and assigns the passed
+     * {@link IChannelControlBlock} to it.
+     *
+     * @param ccb
+     * @return
+     */
+    public IChannelWriteInterface createWriteInterface(IChannelControlBlock ccb);
 }
diff --git a/hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/protocols/muxdemux/IChannelReadInterface.java b/hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/IChannelReadInterface.java
similarity index 61%
rename from hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/protocols/muxdemux/IChannelReadInterface.java
rename to hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/IChannelReadInterface.java
index 8639fb7..357d761 100644
--- a/hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/protocols/muxdemux/IChannelReadInterface.java
+++ b/hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/IChannelReadInterface.java
@@ -16,13 +16,15 @@
  * specific language governing permissions and limitations
  * under the License.
  */
-package org.apache.hyracks.net.protocols.muxdemux;
+package org.apache.hyracks.api.comm;
 
-import org.apache.hyracks.net.buffers.IBufferAcceptor;
-import org.apache.hyracks.net.buffers.ICloseableBufferAcceptor;
+import java.io.IOException;
+import java.nio.channels.SocketChannel;
+
+import org.apache.hyracks.api.exceptions.NetException;
 
 /**
- * Represents the read interface of a {@link ChannelControlBlock}.
+ * Represents the read interface of a {@link IChannelControlBlock}.
  *
  * @author vinayakb
  */
@@ -56,4 +58,42 @@
      *            - the size of each buffer
      */
     public void setBufferFactory(IBufferFactory bufferFactory, int limit, int frameSize);
+
+    /**
+     * Try to read as much as {@code size} bytes from {@code sc}
+     *
+     * @param sc
+     * @param size
+     * @return The number of read bytes.
+     * @throws IOException
+     * @throws NetException
+     */
+    public int read(SocketChannel sc, int size) throws IOException, NetException;
+
+    /**
+     * Sets the read credits of this {@link IChannelReadInterface}
+     *
+     * @param credits
+     */
+    public void setReadCredits(int credits);
+
+    /**
+     * @return The current read credits of this {@link IChannelReadInterface}
+     */
+    public int getCredits();
+
+    /**
+     * Forces the current read buffer to be flushed
+     */
+    public void flush();
+
+    /**
+     * @return The current full buffer acceptor of this {@link IChannelReadInterface}
+     */
+    public ICloseableBufferAcceptor getFullBufferAcceptor();
+
+    /**
+     * @return The buffer factory used by this {@link IChannelReadInterface}
+     */
+    public IBufferFactory getBufferFactory();
 }
diff --git a/hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/protocols/muxdemux/IChannelWriteInterface.java b/hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/IChannelWriteInterface.java
similarity index 64%
rename from hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/protocols/muxdemux/IChannelWriteInterface.java
rename to hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/IChannelWriteInterface.java
index dc38ea4..993dd2c 100644
--- a/hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/protocols/muxdemux/IChannelWriteInterface.java
+++ b/hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/IChannelWriteInterface.java
@@ -16,15 +16,12 @@
  * specific language governing permissions and limitations
  * under the License.
  */
-package org.apache.hyracks.net.protocols.muxdemux;
+package org.apache.hyracks.api.comm;
 
-import org.apache.hyracks.net.buffers.IBufferAcceptor;
-import org.apache.hyracks.net.buffers.ICloseableBufferAcceptor;
+import org.apache.hyracks.api.exceptions.NetException;
 
 /**
- * Represents the write interface of a {@link ChannelControlBlock}.
- *
- * @author vinayakb
+ * Represents the write interface of a {@link IChannelControlBlock}.
  */
 public interface IChannelWriteInterface {
     /**
@@ -56,4 +53,36 @@
      *            - the size of each buffer
      */
     public void setBufferFactory(IBufferFactory bufferFactory, int limit, int frameSize);
+
+    /**
+     * Performs a pending write operation based on the current state of
+     * this {@link IChannelWriteInterface}
+     *
+     * @param writerState
+     * @throws NetException
+     */
+    public void write(IConnectionWriterState writerState) throws NetException;
+
+    /**
+     * Completes the current write operation on this {@link IChannelWriteInterface}
+     */
+    public void writeComplete();
+
+    /**
+     * Add credits to this this {@link IChannelWriteInterface}
+     *
+     * @param credit
+     */
+    public void addCredits(int credit);
+
+    /**
+     * @return The current credits of this {@link IChannelWriteInterface}
+     */
+    public int getCredits();
+
+    /**
+     * Adjusts the {@link IChannelControlBlock} writability based on the current
+     * state of this {@link IChannelWriteInterface}
+     */
+    public void adjustChannelWritability();
 }
diff --git a/hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/buffers/ICloseableBufferAcceptor.java b/hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/ICloseableBufferAcceptor.java
similarity index 96%
rename from hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/buffers/ICloseableBufferAcceptor.java
rename to hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/ICloseableBufferAcceptor.java
index 1697cfd..fba0eaf 100644
--- a/hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/buffers/ICloseableBufferAcceptor.java
+++ b/hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/ICloseableBufferAcceptor.java
@@ -16,7 +16,7 @@
  * specific language governing permissions and limitations
  * under the License.
  */
-package org.apache.hyracks.net.buffers;
+package org.apache.hyracks.api.comm;
 
 /**
  * A buffer acceptor that can be closed to indicate end of transmission or an error code
diff --git a/hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/buffers/IBufferAcceptor.java b/hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/IConnectionWriterState.java
similarity index 65%
copy from hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/buffers/IBufferAcceptor.java
copy to hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/IConnectionWriterState.java
index d7febd2..1f4f204 100644
--- a/hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/buffers/IBufferAcceptor.java
+++ b/hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/IConnectionWriterState.java
@@ -16,20 +16,23 @@
  * specific language governing permissions and limitations
  * under the License.
  */
-package org.apache.hyracks.net.buffers;
+package org.apache.hyracks.api.comm;
 
 import java.nio.ByteBuffer;
 
-/**
- * Accepts buffers.
- *
- * @author vinayakb
- */
-public interface IBufferAcceptor {
+public interface IConnectionWriterState {
+
     /**
-     * Accept a buffer.
+     * Resets the connection write state based on the passed parameters.
      *
-     * @param buffer
+     * @param pendingBuffer
+     * @param pendingWriteSize
+     * @param ccb
      */
-    public void accept(ByteBuffer buffer);
+    public void reset(ByteBuffer pendingBuffer, int pendingWriteSize, IChannelControlBlock ccb);
+
+    /**
+     * @return The command of this {@link IConnectionWriterState}
+     */
+    public MuxDemuxCommand getCommand();
 }
diff --git a/hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/protocols/muxdemux/MuxDemuxCommand.java b/hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/MuxDemuxCommand.java
similarity index 87%
rename from hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/protocols/muxdemux/MuxDemuxCommand.java
rename to hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/MuxDemuxCommand.java
index b53cecf..22b77f7 100644
--- a/hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/protocols/muxdemux/MuxDemuxCommand.java
+++ b/hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/MuxDemuxCommand.java
@@ -16,20 +16,20 @@
  * specific language governing permissions and limitations
  * under the License.
  */
-package org.apache.hyracks.net.protocols.muxdemux;
+package org.apache.hyracks.api.comm;
 
 import java.nio.ByteBuffer;
 
-import org.apache.hyracks.net.exceptions.NetException;
+import org.apache.hyracks.api.exceptions.NetException;
 
-class MuxDemuxCommand {
-    static final int MAX_CHANNEL_ID = Integer.MAX_VALUE - 1;
+public class MuxDemuxCommand {
+    public static final int MAX_CHANNEL_ID = Integer.MAX_VALUE - 1;
 
-    static final int COMMAND_SIZE = 8;
+    public static final int COMMAND_SIZE = 8;
 
-    static final int MAX_DATA_VALUE = 0x1fffffff;
+    public static final int MAX_DATA_VALUE = 0x1fffffff;
 
-    enum CommandType {
+    public enum CommandType {
         OPEN_CHANNEL,
         CLOSE_CHANNEL,
         CLOSE_CHANNEL_ACK,
@@ -90,4 +90,4 @@
     public String toString() {
         return channelId + ":" + type + ":" + data;
     }
-}
+}
\ No newline at end of file
diff --git a/hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/exceptions/NetException.java b/hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/exceptions/NetException.java
similarity index 93%
rename from hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/exceptions/NetException.java
rename to hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/exceptions/NetException.java
index 659e6b5..eaf67af 100644
--- a/hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/exceptions/NetException.java
+++ b/hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/exceptions/NetException.java
@@ -16,12 +16,13 @@
  * specific language governing permissions and limitations
  * under the License.
  */
-package org.apache.hyracks.net.exceptions;
+package org.apache.hyracks.api.exceptions;
 
 public class NetException extends Exception {
     private static final long serialVersionUID = 1L;
 
     public NetException() {
+        // empty constructor
     }
 
     public NetException(String message, Throwable cause) {
diff --git a/hyracks-fullstack/hyracks/hyracks-client/src/main/java/org/apache/hyracks/client/net/ClientNetworkManager.java b/hyracks-fullstack/hyracks/hyracks-client/src/main/java/org/apache/hyracks/client/net/ClientNetworkManager.java
index 6f42410..c238ae3 100644
--- a/hyracks-fullstack/hyracks/hyracks-client/src/main/java/org/apache/hyracks/client/net/ClientNetworkManager.java
+++ b/hyracks-fullstack/hyracks/hyracks-client/src/main/java/org/apache/hyracks/client/net/ClientNetworkManager.java
@@ -22,9 +22,10 @@
 import java.net.InetSocketAddress;
 import java.net.SocketAddress;
 
+import org.apache.hyracks.api.exceptions.NetException;
 import org.apache.hyracks.comm.channels.IChannelConnectionFactory;
-import org.apache.hyracks.net.exceptions.NetException;
 import org.apache.hyracks.net.protocols.muxdemux.ChannelControlBlock;
+import org.apache.hyracks.net.protocols.muxdemux.FullFrameChannelInterfaceFactory;
 import org.apache.hyracks.net.protocols.muxdemux.MultiplexedConnection;
 import org.apache.hyracks.net.protocols.muxdemux.MuxDemux;
 import org.apache.hyracks.net.protocols.muxdemux.MuxDemuxPerformanceCounters;
@@ -38,7 +39,7 @@
         /* This is a connect only socket and does not listen to any incoming connections, so pass null to
          * localAddress and listener.
          */
-        md = new MuxDemux(null, null, nThreads, MAX_CONNECTION_ATTEMPTS);
+        md = new MuxDemux(null, null, nThreads, MAX_CONNECTION_ATTEMPTS, FullFrameChannelInterfaceFactory.INSTANCE);
     }
 
     public void start() throws IOException {
diff --git a/hyracks-fullstack/hyracks/hyracks-comm/src/main/java/org/apache/hyracks/comm/channels/DatasetNetworkInputChannel.java b/hyracks-fullstack/hyracks/hyracks-comm/src/main/java/org/apache/hyracks/comm/channels/DatasetNetworkInputChannel.java
index fba7cf5..e3c6f4a 100644
--- a/hyracks-fullstack/hyracks/hyracks-comm/src/main/java/org/apache/hyracks/comm/channels/DatasetNetworkInputChannel.java
+++ b/hyracks-fullstack/hyracks/hyracks-comm/src/main/java/org/apache/hyracks/comm/channels/DatasetNetworkInputChannel.java
@@ -27,13 +27,13 @@
 
 import org.apache.hyracks.api.channels.IInputChannel;
 import org.apache.hyracks.api.channels.IInputChannelMonitor;
+import org.apache.hyracks.api.comm.IBufferAcceptor;
+import org.apache.hyracks.api.comm.IChannelControlBlock;
+import org.apache.hyracks.api.comm.ICloseableBufferAcceptor;
 import org.apache.hyracks.api.context.IHyracksCommonContext;
 import org.apache.hyracks.api.dataset.ResultSetId;
 import org.apache.hyracks.api.exceptions.HyracksDataException;
 import org.apache.hyracks.api.job.JobId;
-import org.apache.hyracks.net.buffers.IBufferAcceptor;
-import org.apache.hyracks.net.buffers.ICloseableBufferAcceptor;
-import org.apache.hyracks.net.protocols.muxdemux.ChannelControlBlock;
 
 public class DatasetNetworkInputChannel implements IInputChannel {
     private static final Logger LOGGER = Logger.getLogger(DatasetNetworkInputChannel.class.getName());
@@ -54,7 +54,7 @@
 
     private final int nBuffers;
 
-    private ChannelControlBlock ccb;
+    private IChannelControlBlock ccb;
 
     private IInputChannelMonitor monitor;
 
diff --git a/hyracks-fullstack/hyracks/hyracks-comm/src/main/java/org/apache/hyracks/comm/channels/IChannelConnectionFactory.java b/hyracks-fullstack/hyracks/hyracks-comm/src/main/java/org/apache/hyracks/comm/channels/IChannelConnectionFactory.java
index 2e66f54..fbcb0da 100644
--- a/hyracks-fullstack/hyracks/hyracks-comm/src/main/java/org/apache/hyracks/comm/channels/IChannelConnectionFactory.java
+++ b/hyracks-fullstack/hyracks/hyracks-comm/src/main/java/org/apache/hyracks/comm/channels/IChannelConnectionFactory.java
@@ -20,9 +20,9 @@
 
 import java.net.SocketAddress;
 
-import org.apache.hyracks.net.exceptions.NetException;
-import org.apache.hyracks.net.protocols.muxdemux.ChannelControlBlock;
+import org.apache.hyracks.api.comm.IChannelControlBlock;
+import org.apache.hyracks.api.exceptions.NetException;
 
 public interface IChannelConnectionFactory {
-    public ChannelControlBlock connect(SocketAddress remoteAddress) throws InterruptedException, NetException;
+    public IChannelControlBlock connect(SocketAddress remoteAddress) throws InterruptedException, NetException;
 }
diff --git a/hyracks-fullstack/hyracks/hyracks-comm/src/main/java/org/apache/hyracks/comm/channels/NetworkInputChannel.java b/hyracks-fullstack/hyracks/hyracks-comm/src/main/java/org/apache/hyracks/comm/channels/NetworkInputChannel.java
index 235536f..a846da3 100644
--- a/hyracks-fullstack/hyracks/hyracks-comm/src/main/java/org/apache/hyracks/comm/channels/NetworkInputChannel.java
+++ b/hyracks-fullstack/hyracks/hyracks-comm/src/main/java/org/apache/hyracks/comm/channels/NetworkInputChannel.java
@@ -27,12 +27,12 @@
 
 import org.apache.hyracks.api.channels.IInputChannel;
 import org.apache.hyracks.api.channels.IInputChannelMonitor;
+import org.apache.hyracks.api.comm.IBufferAcceptor;
+import org.apache.hyracks.api.comm.IChannelControlBlock;
+import org.apache.hyracks.api.comm.ICloseableBufferAcceptor;
 import org.apache.hyracks.api.context.IHyracksCommonContext;
 import org.apache.hyracks.api.exceptions.HyracksDataException;
 import org.apache.hyracks.api.partitions.PartitionId;
-import org.apache.hyracks.net.buffers.IBufferAcceptor;
-import org.apache.hyracks.net.buffers.ICloseableBufferAcceptor;
-import org.apache.hyracks.net.protocols.muxdemux.ChannelControlBlock;
 
 public class NetworkInputChannel implements IInputChannel {
     private static final Logger LOGGER = Logger.getLogger(NetworkInputChannel.class.getName());
@@ -49,7 +49,7 @@
 
     private final int nBuffers;
 
-    private ChannelControlBlock ccb;
+    private IChannelControlBlock ccb;
 
     private IInputChannelMonitor monitor;
 
@@ -99,7 +99,8 @@
         }
         ccb.getReadInterface().setFullBufferAcceptor(new ReadFullBufferAcceptor());
         ccb.getWriteInterface().setEmptyBufferAcceptor(new WriteEmptyBufferAcceptor());
-        ccb.getReadInterface().setBufferFactory(new ReadBufferFactory(nBuffers, ctx), nBuffers, ctx.getInitialFrameSize());
+        ccb.getReadInterface().setBufferFactory(new ReadBufferFactory(nBuffers, ctx), nBuffers,
+                ctx.getInitialFrameSize());
         ByteBuffer writeBuffer = ByteBuffer.allocate(INITIAL_MESSAGE_SIZE);
         writeBuffer.putLong(partitionId.getJobId().getId());
         writeBuffer.putInt(partitionId.getConnectorDescriptorId().getId());
diff --git a/hyracks-fullstack/hyracks/hyracks-comm/src/main/java/org/apache/hyracks/comm/channels/NetworkOutputChannel.java b/hyracks-fullstack/hyracks/hyracks-comm/src/main/java/org/apache/hyracks/comm/channels/NetworkOutputChannel.java
index 46220de..60e2e35 100644
--- a/hyracks-fullstack/hyracks/hyracks-comm/src/main/java/org/apache/hyracks/comm/channels/NetworkOutputChannel.java
+++ b/hyracks-fullstack/hyracks/hyracks-comm/src/main/java/org/apache/hyracks/comm/channels/NetworkOutputChannel.java
@@ -22,9 +22,9 @@
 import java.util.ArrayDeque;
 import java.util.Deque;
 
+import org.apache.hyracks.api.comm.IBufferAcceptor;
 import org.apache.hyracks.api.comm.IFrameWriter;
 import org.apache.hyracks.api.exceptions.HyracksDataException;
-import org.apache.hyracks.net.buffers.IBufferAcceptor;
 import org.apache.hyracks.net.protocols.muxdemux.ChannelControlBlock;
 
 public class NetworkOutputChannel implements IFrameWriter {
diff --git a/hyracks-fullstack/hyracks/hyracks-comm/src/main/java/org/apache/hyracks/comm/channels/ReadBufferFactory.java b/hyracks-fullstack/hyracks/hyracks-comm/src/main/java/org/apache/hyracks/comm/channels/ReadBufferFactory.java
index 3aa77b9..876b3de 100644
--- a/hyracks-fullstack/hyracks/hyracks-comm/src/main/java/org/apache/hyracks/comm/channels/ReadBufferFactory.java
+++ b/hyracks-fullstack/hyracks/hyracks-comm/src/main/java/org/apache/hyracks/comm/channels/ReadBufferFactory.java
@@ -20,12 +20,9 @@
 
 import java.nio.ByteBuffer;
 
+import org.apache.hyracks.api.comm.IBufferFactory;
 import org.apache.hyracks.api.context.IHyracksCommonContext;
-import org.apache.hyracks.net.protocols.muxdemux.IBufferFactory;
 
-/**
- * @author yingyib
- */
 public class ReadBufferFactory implements IBufferFactory {
 
     private final int limit;
@@ -39,17 +36,12 @@
 
     @Override
     public ByteBuffer createBuffer() {
-        try {
-            if (counter >= limit) {
-                return null;
-            } else {
-                ByteBuffer frame = ByteBuffer.allocate(frameSize);
-                counter++;
-                return frame;
-            }
-        } catch (Exception e) {
-            throw new IllegalStateException(e);
+        if (counter >= limit) {
+            return null;
+        } else {
+            ByteBuffer frame = ByteBuffer.allocate(frameSize);
+            counter++;
+            return frame;
         }
     }
-
 }
diff --git a/hyracks-fullstack/hyracks/hyracks-control/hyracks-control-cc/src/main/java/org/apache/hyracks/control/cc/NodeControllerState.java b/hyracks-fullstack/hyracks/hyracks-control/hyracks-control-cc/src/main/java/org/apache/hyracks/control/cc/NodeControllerState.java
index d93ba0d..ff5832a 100644
--- a/hyracks-fullstack/hyracks/hyracks-control/hyracks-control-cc/src/main/java/org/apache/hyracks/control/cc/NodeControllerState.java
+++ b/hyracks-fullstack/hyracks/hyracks-control/hyracks-control-cc/src/main/java/org/apache/hyracks/control/cc/NodeControllerState.java
@@ -23,10 +23,6 @@
 import java.util.Map;
 import java.util.Set;
 
-import org.json.JSONArray;
-import org.json.JSONException;
-import org.json.JSONObject;
-
 import org.apache.hyracks.api.comm.NetworkAddress;
 import org.apache.hyracks.api.job.JobId;
 import org.apache.hyracks.control.common.base.INodeController;
@@ -35,6 +31,9 @@
 import org.apache.hyracks.control.common.heartbeat.HeartbeatData;
 import org.apache.hyracks.control.common.heartbeat.HeartbeatSchema;
 import org.apache.hyracks.control.common.heartbeat.HeartbeatSchema.GarbageCollectorInfo;
+import org.json.JSONArray;
+import org.json.JSONException;
+import org.json.JSONObject;
 
 public class NodeControllerState {
     private static final int RRD_SIZE = 720;
@@ -46,6 +45,8 @@
     private final NetworkAddress dataPort;
 
     private final NetworkAddress datasetPort;
+
+    private final NetworkAddress messagingPort;
 
     private final Set<JobId> activeJobIds;
 
@@ -142,6 +143,7 @@
         ncConfig = reg.getNCConfig();
         dataPort = reg.getDataPort();
         datasetPort = reg.getDatasetPort();
+        messagingPort = reg.getMessagingPort();
         activeJobIds = new HashSet<JobId>();
 
         osName = reg.getOSName();
@@ -264,6 +266,10 @@
         return datasetPort;
     }
 
+    public NetworkAddress getMessagingPort() {
+        return messagingPort;
+    }
+
     public JSONObject toSummaryJSON() throws JSONException {
         JSONObject o = new JSONObject();
         o.put("node-id", ncConfig.nodeId);
diff --git a/hyracks-fullstack/hyracks/hyracks-control/hyracks-control-cc/src/main/java/org/apache/hyracks/control/cc/work/GetNodeControllersInfoWork.java b/hyracks-fullstack/hyracks/hyracks-control/hyracks-control-cc/src/main/java/org/apache/hyracks/control/cc/work/GetNodeControllersInfoWork.java
index 785a202..726bf12 100644
--- a/hyracks-fullstack/hyracks/hyracks-control/hyracks-control-cc/src/main/java/org/apache/hyracks/control/cc/work/GetNodeControllersInfoWork.java
+++ b/hyracks-fullstack/hyracks/hyracks-control/hyracks-control-cc/src/main/java/org/apache/hyracks/control/cc/work/GetNodeControllersInfoWork.java
@@ -40,11 +40,11 @@
 
     @Override
     public void run() {
-        Map<String, NodeControllerInfo> result = new LinkedHashMap<String, NodeControllerInfo>();
+        Map<String, NodeControllerInfo> result = new LinkedHashMap<>();
         Map<String, NodeControllerState> nodeMap = ccs.getNodeMap();
         for (Map.Entry<String, NodeControllerState> e : nodeMap.entrySet()) {
-            result.put(e.getKey(), new NodeControllerInfo(e.getKey(), NodeStatus.ALIVE, e.getValue().getDataPort(), e
-                    .getValue().getDatasetPort()));
+            result.put(e.getKey(), new NodeControllerInfo(e.getKey(), NodeStatus.ALIVE, e.getValue().getDataPort(),
+                    e.getValue().getDatasetPort(), e.getValue().getMessagingPort()));
         }
         callback.setValue(result);
     }
diff --git a/hyracks-fullstack/hyracks/hyracks-control/hyracks-control-common/src/main/java/org/apache/hyracks/control/common/controllers/NCConfig.java b/hyracks-fullstack/hyracks/hyracks-control/hyracks-control-common/src/main/java/org/apache/hyracks/control/common/controllers/NCConfig.java
index d08df60..64b9737 100644
--- a/hyracks-fullstack/hyracks/hyracks-control/hyracks-control-common/src/main/java/org/apache/hyracks/control/common/controllers/NCConfig.java
+++ b/hyracks-fullstack/hyracks/hyracks-control/hyracks-control-common/src/main/java/org/apache/hyracks/control/common/controllers/NCConfig.java
@@ -18,18 +18,18 @@
  */
 package org.apache.hyracks.control.common.controllers;
 
+import java.io.IOException;
+import java.io.Serializable;
+import java.net.InetAddress;
+import java.util.List;
+import java.util.Map;
+
 import org.apache.hyracks.api.application.IApplicationConfig;
 import org.apache.hyracks.control.common.application.IniApplicationConfig;
 import org.ini4j.Ini;
 import org.kohsuke.args4j.Argument;
 import org.kohsuke.args4j.Option;
 import org.kohsuke.args4j.spi.StopOptionHandler;
-
-import java.io.IOException;
-import java.io.Serializable;
-import java.net.InetAddress;
-import java.util.List;
-import java.util.Map;
 
 public class NCConfig implements Serializable {
     private static final long serialVersionUID = 2L;
@@ -82,7 +82,7 @@
     @Option(name = "-result-public-port", usage = "Public IP port to announce dataset result distribution listener (default: same as -result-port; must set -result-public-ip-address also)", required = false)
     public int resultPublicPort = 0;
 
-    @Option(name = "-retries", usage ="Number of attempts to contact CC before giving up (default = 5)")
+    @Option(name = "-retries", usage = "Number of attempts to contact CC before giving up (default = 5)")
     public int retries = 5;
 
     @Option(name = "-iodevices", usage = "Comma separated list of IO Device mount points (default: One device in default temp folder)", required = false)
@@ -112,6 +112,23 @@
     @Option(name = "-config-file", usage = "Specify path to local configuration file (default: no local config)", required = false)
     public String configFile = null;
 
+    //TODO add messaging values to NC start scripts
+    @Option(name = "-messaging-ip-address", usage = "IP Address to bind messaging "
+            + "listener (default: same as -address)", required = false)
+    public String messagingIPAddress;
+
+    @Option(name = "-messaging-port", usage = "IP port to bind messaging listener "
+            + "(default: random port)", required = false)
+    public int messagingPort = 0;
+
+    @Option(name = "-messaging-public-ip-address", usage = "Public IP Address to announce messaging"
+            + " listener (default: same as -messaging-ip-address)", required = false)
+    public String messagingPublicIPAddress;
+
+    @Option(name = "-messaging-public-port", usage = "Public IP port to announce messaging listener"
+            + " (default: same as -messaging-port; must set -messaging-public-port also)", required = false)
+    public int messagingPublicPort = 0;
+
     @Argument
     @Option(name = "--", handler = StopOptionHandler.class)
     public List<String> appArgs;
@@ -139,13 +156,14 @@
         resultIPAddress = IniUtils.getString(ini, nodeSection, "result.address", resultIPAddress);
         resultPort = IniUtils.getInt(ini, nodeSection, "result.port", resultPort);
 
-        clusterNetPublicIPAddress = IniUtils.getString(
-                ini, nodeSection, "public.cluster.address", clusterNetPublicIPAddress);
+        clusterNetPublicIPAddress = IniUtils.getString(ini, nodeSection, "public.cluster.address",
+                clusterNetPublicIPAddress);
         clusterNetPublicPort = IniUtils.getInt(ini, nodeSection, "public.cluster.port", clusterNetPublicPort);
         dataPublicIPAddress = IniUtils.getString(ini, nodeSection, "public.data.address", dataPublicIPAddress);
         dataPublicPort = IniUtils.getInt(ini, nodeSection, "public.data.port", dataPublicPort);
         resultPublicIPAddress = IniUtils.getString(ini, nodeSection, "public.result.address", resultPublicIPAddress);
         resultPublicPort = IniUtils.getInt(ini, nodeSection, "public.result.port", resultPublicPort);
+        //TODO pass messaging info from ini file
 
         retries = IniUtils.getInt(ini, nodeSection, "retries", retries);
 
@@ -169,23 +187,41 @@
         }
 
         // "address" is the default for all IP addresses
-        if (clusterNetIPAddress == null) clusterNetIPAddress = ipAddress;
-        if (dataIPAddress == null) dataIPAddress = ipAddress;
-        if (resultIPAddress == null) resultIPAddress = ipAddress;
+        if (clusterNetIPAddress == null) {
+            clusterNetIPAddress = ipAddress;
+        }
+        if (dataIPAddress == null) {
+            dataIPAddress = ipAddress;
+        }
+        if (resultIPAddress == null) {
+            resultIPAddress = ipAddress;
+        }
 
         // All "public" options default to their "non-public" versions
-        if (clusterNetPublicIPAddress == null) clusterNetPublicIPAddress = clusterNetIPAddress;
-        if (clusterNetPublicPort == 0) clusterNetPublicPort = clusterNetPort;
-        if (dataPublicIPAddress == null) dataPublicIPAddress = dataIPAddress;
-        if (dataPublicPort == 0) dataPublicPort = dataPort;
-        if (resultPublicIPAddress == null) resultPublicIPAddress = resultIPAddress;
-        if (resultPublicPort == 0) resultPublicPort = resultPort;
+        if (clusterNetPublicIPAddress == null) {
+            clusterNetPublicIPAddress = clusterNetIPAddress;
+        }
+        if (clusterNetPublicPort == 0) {
+            clusterNetPublicPort = clusterNetPort;
+        }
+        if (dataPublicIPAddress == null) {
+            dataPublicIPAddress = dataIPAddress;
+        }
+        if (dataPublicPort == 0) {
+            dataPublicPort = dataPort;
+        }
+        if (resultPublicIPAddress == null) {
+            resultPublicIPAddress = resultIPAddress;
+        }
+        if (resultPublicPort == 0) {
+            resultPublicPort = resultPort;
+        }
     }
 
     /**
      * @return An IApplicationConfig representing this NCConfig.
-     * Note: Currently this only includes the values from the configuration
-     * file, not anything specified on the command-line. QQQ
+     *         Note: Currently this only includes the values from the configuration
+     *         file, not anything specified on the command-line. QQQ
      */
     public IApplicationConfig getAppConfig() {
         return new IniApplicationConfig(ini);
@@ -215,10 +251,12 @@
         configuration.put("result-time-to-live", String.valueOf(resultTTL));
         configuration.put("result-sweep-threshold", String.valueOf(resultSweepThreshold));
         configuration.put("result-manager-memory", String.valueOf(resultManagerMemory));
-
+        configuration.put("messaging-ip-address", messagingIPAddress);
+        configuration.put("messaging-port", String.valueOf(messagingPort));
+        configuration.put("messaging-public-ip-address", messagingPublicIPAddress);
+        configuration.put("messaging-public-port", String.valueOf(messagingPublicPort));
         if (appNCMainClass != null) {
             configuration.put("app-nc-main-class", appNCMainClass);
         }
-
     }
 }
diff --git a/hyracks-fullstack/hyracks/hyracks-control/hyracks-control-common/src/main/java/org/apache/hyracks/control/common/controllers/NodeRegistration.java b/hyracks-fullstack/hyracks/hyracks-control/hyracks-control-common/src/main/java/org/apache/hyracks/control/common/controllers/NodeRegistration.java
index 5a23455..bb8022e 100644
--- a/hyracks-fullstack/hyracks/hyracks-control/hyracks-control-common/src/main/java/org/apache/hyracks/control/common/controllers/NodeRegistration.java
+++ b/hyracks-fullstack/hyracks/hyracks-control/hyracks-control-common/src/main/java/org/apache/hyracks/control/common/controllers/NodeRegistration.java
@@ -65,10 +65,13 @@
 
     private final HeartbeatSchema hbSchema;
 
+    private final NetworkAddress messagingPort;
+
     public NodeRegistration(InetSocketAddress ncAddress, String nodeId, NCConfig ncConfig, NetworkAddress dataPort,
             NetworkAddress datasetPort, String osName, String arch, String osVersion, int nProcessors, String vmName,
             String vmVersion, String vmVendor, String classpath, String libraryPath, String bootClasspath,
-            List<String> inputArguments, Map<String, String> systemProperties, HeartbeatSchema hbSchema) {
+            List<String> inputArguments, Map<String, String> systemProperties, HeartbeatSchema hbSchema,
+            NetworkAddress messagingPort) {
         this.ncAddress = ncAddress;
         this.nodeId = nodeId;
         this.ncConfig = ncConfig;
@@ -87,6 +90,7 @@
         this.inputArguments = inputArguments;
         this.systemProperties = systemProperties;
         this.hbSchema = hbSchema;
+        this.messagingPort = messagingPort;
     }
 
     public InetSocketAddress getNodeControllerAddress() {
@@ -160,4 +164,8 @@
     public Map<String, String> getSystemProperties() {
         return systemProperties;
     }
+
+    public NetworkAddress getMessagingPort() {
+        return messagingPort;
+    }
 }
diff --git a/hyracks-fullstack/hyracks/hyracks-control/hyracks-control-nc/src/main/java/org/apache/hyracks/control/nc/NCDriver.java b/hyracks-fullstack/hyracks/hyracks-control/hyracks-control-nc/src/main/java/org/apache/hyracks/control/nc/NCDriver.java
index b015e3d..2d0a49d 100644
--- a/hyracks-fullstack/hyracks/hyracks-control/hyracks-control-nc/src/main/java/org/apache/hyracks/control/nc/NCDriver.java
+++ b/hyracks-fullstack/hyracks/hyracks-control/hyracks-control-nc/src/main/java/org/apache/hyracks/control/nc/NCDriver.java
@@ -18,12 +18,12 @@
  */
 package org.apache.hyracks.control.nc;
 
-import org.apache.hyracks.control.common.controllers.NCConfig;
-import org.kohsuke.args4j.CmdLineParser;
-
 import java.util.logging.Level;
 import java.util.logging.Logger;
 
+import org.apache.hyracks.control.common.controllers.NCConfig;
+import org.kohsuke.args4j.CmdLineParser;
+
 public class NCDriver {
     private static final Logger LOGGER = Logger.getLogger(NCDriver.class.getName());
 
diff --git a/hyracks-fullstack/hyracks/hyracks-control/hyracks-control-nc/src/main/java/org/apache/hyracks/control/nc/NodeControllerService.java b/hyracks-fullstack/hyracks/hyracks-control/hyracks-control-nc/src/main/java/org/apache/hyracks/control/nc/NodeControllerService.java
index 43cac74..ac994a6 100644
--- a/hyracks-fullstack/hyracks/hyracks-control/hyracks-control-nc/src/main/java/org/apache/hyracks/control/nc/NodeControllerService.java
+++ b/hyracks-fullstack/hyracks/hyracks-control/hyracks-control-nc/src/main/java/org/apache/hyracks/control/nc/NodeControllerService.java
@@ -73,6 +73,7 @@
 import org.apache.hyracks.control.nc.io.profiling.IIOCounter;
 import org.apache.hyracks.control.nc.io.profiling.IOCounterFactory;
 import org.apache.hyracks.control.nc.net.DatasetNetworkManager;
+import org.apache.hyracks.control.nc.net.MessagingNetworkManager;
 import org.apache.hyracks.control.nc.net.NetworkManager;
 import org.apache.hyracks.control.nc.partitions.PartitionManager;
 import org.apache.hyracks.control.nc.resources.memory.MemoryManager;
@@ -91,6 +92,7 @@
 import org.apache.hyracks.ipc.api.IIPCI;
 import org.apache.hyracks.ipc.api.IPCPerformanceCounters;
 import org.apache.hyracks.ipc.impl.IPCSystem;
+import org.apache.hyracks.net.protocols.muxdemux.FullFrameChannelInterfaceFactory;
 import org.apache.hyracks.net.protocols.muxdemux.MuxDemuxPerformanceCounters;
 
 public class NodeControllerService implements IControllerService {
@@ -158,6 +160,8 @@
 
     private IIOCounter ioCounter;
 
+    private MessagingNetworkManager messagingNetManager;
+
     public NodeControllerService(NCConfig ncConfig) throws Exception {
         this.ncConfig = ncConfig;
         id = ncConfig.nodeId;
@@ -171,7 +175,8 @@
         }
         partitionManager = new PartitionManager(this);
         netManager = new NetworkManager(ncConfig.dataIPAddress, ncConfig.dataPort, partitionManager,
-                ncConfig.nNetThreads, ncConfig.nNetBuffers, ncConfig.dataPublicIPAddress, ncConfig.dataPublicPort);
+                ncConfig.nNetThreads, ncConfig.nNetBuffers, ncConfig.dataPublicIPAddress, ncConfig.dataPublicPort,
+                FullFrameChannelInterfaceFactory.INSTANCE);
 
         lccm = new LifeCycleComponentManager();
         queue = new WorkQueue(Thread.NORM_PRIORITY); // Reserves MAX_PRIORITY of the heartbeat thread.
@@ -247,7 +252,12 @@
                 ncConfig.resultTTL, ncConfig.resultSweepThreshold);
         datasetNetworkManager = new DatasetNetworkManager(ncConfig.resultIPAddress, ncConfig.resultPort,
                 datasetPartitionManager, ncConfig.nNetThreads, ncConfig.nNetBuffers, ncConfig.resultPublicIPAddress,
-                ncConfig.resultPublicPort);
+                ncConfig.resultPublicPort, FullFrameChannelInterfaceFactory.INSTANCE);
+        if (ncConfig.messagingIPAddress != null && appCtx.getMessagingChannelInterfaceFactory() != null) {
+            messagingNetManager = new MessagingNetworkManager(this, ncConfig.messagingIPAddress, ncConfig.messagingPort,
+                    ncConfig.nNetThreads, ncConfig.messagingPublicIPAddress, ncConfig.messagingPublicPort,
+                    appCtx.getMessagingChannelInterfaceFactory());
+        }
     }
 
     @Override
@@ -260,8 +270,11 @@
         init();
 
         datasetNetworkManager.start();
-        IIPCHandle ccIPCHandle =
-                ipc.getHandle(new InetSocketAddress(ncConfig.ccHost, ncConfig.ccPort), ncConfig.retries);
+        if (messagingNetManager != null) {
+            messagingNetManager.start();
+        }
+        IIPCHandle ccIPCHandle = ipc.getHandle(new InetSocketAddress(ncConfig.ccHost, ncConfig.ccPort),
+                ncConfig.retries);
         this.ccs = new ClusterControllerRemoteProxy(ccIPCHandle);
         HeartbeatSchema.GarbageCollectorInfo[] gcInfos = new HeartbeatSchema.GarbageCollectorInfo[gcMXBeans.size()];
         for (int i = 0; i < gcInfos.length; ++i) {
@@ -271,11 +284,13 @@
         // Use "public" versions of network addresses and ports
         NetworkAddress datasetAddress = datasetNetworkManager.getPublicNetworkAddress();
         NetworkAddress netAddress = netManager.getPublicNetworkAddress();
+        NetworkAddress meesagingPort = messagingNetManager != null ? messagingNetManager.getPublicNetworkAddress()
+                : null;
         ccs.registerNode(new NodeRegistration(ipc.getSocketAddress(), id, ncConfig, netAddress, datasetAddress,
                 osMXBean.getName(), osMXBean.getArch(), osMXBean.getVersion(), osMXBean.getAvailableProcessors(),
                 runtimeMXBean.getVmName(), runtimeMXBean.getVmVersion(), runtimeMXBean.getVmVendor(),
                 runtimeMXBean.getClassPath(), runtimeMXBean.getLibraryPath(), runtimeMXBean.getBootClassPath(),
-                runtimeMXBean.getInputArguments(), runtimeMXBean.getSystemProperties(), hbSchema));
+                runtimeMXBean.getInputArguments(), runtimeMXBean.getSystemProperties(), hbSchema, meesagingPort));
 
         synchronized (this) {
             while (registrationPending) {
@@ -338,6 +353,9 @@
             datasetPartitionManager.close();
             netManager.stop();
             datasetNetworkManager.stop();
+            if (messagingNetManager != null) {
+                messagingNetManager.stop();
+            }
             queue.stop();
             if (ncAppEntryPoint != null) {
                 ncAppEntryPoint.stop();
@@ -490,8 +508,7 @@
             CCNCFunctions.Function fn = (CCNCFunctions.Function) payload;
             switch (fn.getFunctionId()) {
                 case SEND_APPLICATION_MESSAGE: {
-                    CCNCFunctions.SendApplicationMessageFunction amf =
-                            (CCNCFunctions.SendApplicationMessageFunction) fn;
+                    CCNCFunctions.SendApplicationMessageFunction amf = (CCNCFunctions.SendApplicationMessageFunction) fn;
                     queue.schedule(new ApplicationMessageWork(NodeControllerService.this, amf.getMessage(),
                             amf.getDeploymentId(), amf.getNodeId()));
                     return;
@@ -516,8 +533,7 @@
                 }
 
                 case REPORT_PARTITION_AVAILABILITY: {
-                    CCNCFunctions.ReportPartitionAvailabilityFunction rpaf =
-                            (CCNCFunctions.ReportPartitionAvailabilityFunction) fn;
+                    CCNCFunctions.ReportPartitionAvailabilityFunction rpaf = (CCNCFunctions.ReportPartitionAvailabilityFunction) fn;
                     queue.schedule(new ReportPartitionAvailabilityWork(NodeControllerService.this,
                             rpaf.getPartitionId(), rpaf.getNetworkAddress()));
                     return;
@@ -530,8 +546,7 @@
                 }
 
                 case GET_NODE_CONTROLLERS_INFO_RESPONSE: {
-                    CCNCFunctions.GetNodeControllersInfoResponseFunction gncirf =
-                            (CCNCFunctions.GetNodeControllersInfoResponseFunction) fn;
+                    CCNCFunctions.GetNodeControllersInfoResponseFunction gncirf = (CCNCFunctions.GetNodeControllersInfoResponseFunction) fn;
                     setNodeControllersInfo(gncirf.getNodeControllerInfos());
                     return;
                 }
@@ -572,6 +587,10 @@
         return datasetPartitionManager;
     }
 
+    public MessagingNetworkManager getMessagingNetworkManager() {
+        return messagingNetManager;
+    }
+
     /**
      * Shutdown hook that invokes {@link NCApplicationEntryPoint#stop() stop} method.
      */
diff --git a/hyracks-fullstack/hyracks/hyracks-control/hyracks-control-nc/src/main/java/org/apache/hyracks/control/nc/application/NCApplicationContext.java b/hyracks-fullstack/hyracks/hyracks-control/hyracks-control-nc/src/main/java/org/apache/hyracks/control/nc/application/NCApplicationContext.java
index d23c701..a2c3f4c 100644
--- a/hyracks-fullstack/hyracks/hyracks-control/hyracks-control-nc/src/main/java/org/apache/hyracks/control/nc/application/NCApplicationContext.java
+++ b/hyracks-fullstack/hyracks/hyracks-control/hyracks-control-nc/src/main/java/org/apache/hyracks/control/nc/application/NCApplicationContext.java
@@ -25,6 +25,7 @@
 import org.apache.hyracks.api.application.IApplicationConfig;
 import org.apache.hyracks.api.application.INCApplicationContext;
 import org.apache.hyracks.api.application.IStateDumpHandler;
+import org.apache.hyracks.api.comm.IChannelInterfaceFactory;
 import org.apache.hyracks.api.context.IHyracksRootContext;
 import org.apache.hyracks.api.lifecycle.ILifeCycleComponentManager;
 import org.apache.hyracks.api.resources.memory.IMemoryManager;
@@ -42,10 +43,11 @@
     private Object appObject;
     private IStateDumpHandler sdh;
     private final NodeControllerService ncs;
+    private IChannelInterfaceFactory messagingChannelInterfaceFactory;
 
-    public NCApplicationContext(NodeControllerService ncs, ServerContext serverCtx, IHyracksRootContext rootCtx, String nodeId,
-                                MemoryManager memoryManager, ILifeCycleComponentManager lifeCyclecomponentManager,
-                                IApplicationConfig appConfig) throws IOException {
+    public NCApplicationContext(NodeControllerService ncs, ServerContext serverCtx, IHyracksRootContext rootCtx,
+            String nodeId, MemoryManager memoryManager, ILifeCycleComponentManager lifeCyclecomponentManager,
+            IApplicationConfig appConfig) throws IOException {
         super(serverCtx, appConfig);
         this.lccm = lifeCyclecomponentManager;
         this.nodeId = nodeId;
@@ -108,4 +110,14 @@
     public IControllerService getControllerService() {
         return ncs;
     }
+
+    @Override
+    public IChannelInterfaceFactory getMessagingChannelInterfaceFactory() {
+        return messagingChannelInterfaceFactory;
+    }
+
+    @Override
+    public void setMessagingChannelInterfaceFactory(IChannelInterfaceFactory interfaceFactory) {
+        this.messagingChannelInterfaceFactory = interfaceFactory;
+    }
 }
diff --git a/hyracks-fullstack/hyracks/hyracks-control/hyracks-control-nc/src/main/java/org/apache/hyracks/control/nc/net/DatasetNetworkManager.java b/hyracks-fullstack/hyracks/hyracks-control/hyracks-control-nc/src/main/java/org/apache/hyracks/control/nc/net/DatasetNetworkManager.java
index 3c99ef3..0b74806 100644
--- a/hyracks-fullstack/hyracks/hyracks-control/hyracks-control-nc/src/main/java/org/apache/hyracks/control/nc/net/DatasetNetworkManager.java
+++ b/hyracks-fullstack/hyracks/hyracks-control/hyracks-control-nc/src/main/java/org/apache/hyracks/control/nc/net/DatasetNetworkManager.java
@@ -25,15 +25,16 @@
 import java.util.logging.Level;
 import java.util.logging.Logger;
 
+import org.apache.hyracks.api.comm.IChannelInterfaceFactory;
+import org.apache.hyracks.api.comm.ICloseableBufferAcceptor;
 import org.apache.hyracks.api.comm.NetworkAddress;
 import org.apache.hyracks.api.dataset.IDatasetPartitionManager;
 import org.apache.hyracks.api.dataset.ResultSetId;
 import org.apache.hyracks.api.exceptions.HyracksException;
+import org.apache.hyracks.api.exceptions.NetException;
 import org.apache.hyracks.api.job.JobId;
 import org.apache.hyracks.comm.channels.IChannelConnectionFactory;
 import org.apache.hyracks.comm.channels.NetworkOutputChannel;
-import org.apache.hyracks.net.buffers.ICloseableBufferAcceptor;
-import org.apache.hyracks.net.exceptions.NetException;
 import org.apache.hyracks.net.protocols.muxdemux.ChannelControlBlock;
 import org.apache.hyracks.net.protocols.muxdemux.IChannelOpenListener;
 import org.apache.hyracks.net.protocols.muxdemux.MultiplexedConnection;
@@ -58,19 +59,24 @@
     private NetworkAddress publicNetworkAddress;
 
     /**
-     * @param inetAddress - Internet address to bind the listen port to
-     * @param inetPort - Port to bind on inetAddress
-     * @param publicInetAddress - Internet address to report to consumers;
-     *    useful when behind NAT. null = same as inetAddress
-     * @param publicInetPort - Port to report to consumers; useful when
-     *    behind NAT. Ignored if publicInetAddress is null. 0 = same as inetPort
+     * @param inetAddress
+     *            - Internet address to bind the listen port to
+     * @param inetPort
+     *            - Port to bind on inetAddress
+     * @param publicInetAddress
+     *            - Internet address to report to consumers;
+     *            useful when behind NAT. null = same as inetAddress
+     * @param publicInetPort
+     *            - Port to report to consumers; useful when
+     *            behind NAT. Ignored if publicInetAddress is null. 0 = same as inetPort
      */
-    public DatasetNetworkManager(String inetAddress, int inetPort, IDatasetPartitionManager partitionManager, int nThreads,
-                                 int nBuffers, String publicInetAddress, int publicInetPort) throws IOException {
+    public DatasetNetworkManager(String inetAddress, int inetPort, IDatasetPartitionManager partitionManager,
+            int nThreads, int nBuffers, String publicInetAddress, int publicInetPort,
+            IChannelInterfaceFactory channelInterfaceFactory) {
         this.partitionManager = partitionManager;
         this.nBuffers = nBuffers;
         md = new MuxDemux(new InetSocketAddress(inetAddress, inetPort), new ChannelOpenListener(), nThreads,
-                MAX_CONNECTION_ATTEMPTS);
+                MAX_CONNECTION_ATTEMPTS, channelInterfaceFactory);
         // Just save these values for the moment; may be reset in start()
         publicNetworkAddress = new NetworkAddress(publicInetAddress, publicInetPort);
     }
@@ -84,12 +90,10 @@
         // make it a copy of localNetworkAddress
         if (publicNetworkAddress.getAddress() == null) {
             publicNetworkAddress = localNetworkAddress;
-        }
-        else {
+        } else {
             // Likewise for public port
             if (publicNetworkAddress.getPort() == 0) {
-                publicNetworkAddress = new NetworkAddress
-                    (publicNetworkAddress.getAddress(), sockAddr.getPort());
+                publicNetworkAddress = new NetworkAddress(publicNetworkAddress.getAddress(), sockAddr.getPort());
             }
         }
     }
diff --git a/hyracks-fullstack/hyracks/hyracks-control/hyracks-control-nc/src/main/java/org/apache/hyracks/control/nc/net/MessagingNetworkManager.java b/hyracks-fullstack/hyracks/hyracks-control/hyracks-control-nc/src/main/java/org/apache/hyracks/control/nc/net/MessagingNetworkManager.java
new file mode 100644
index 0000000..7983b93
--- /dev/null
+++ b/hyracks-fullstack/hyracks/hyracks-control/hyracks-control-nc/src/main/java/org/apache/hyracks/control/nc/net/MessagingNetworkManager.java
@@ -0,0 +1,228 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements.  See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership.  The ASF licenses this file
+ * to you 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.apache.hyracks.control.nc.net;
+
+import java.io.IOException;
+import java.net.InetAddress;
+import java.net.InetSocketAddress;
+import java.net.SocketAddress;
+import java.nio.ByteBuffer;
+import java.util.HashMap;
+import java.util.Map;
+import java.util.logging.Level;
+import java.util.logging.Logger;
+
+import org.apache.hyracks.api.client.NodeControllerInfo;
+import org.apache.hyracks.api.comm.IChannelControlBlock;
+import org.apache.hyracks.api.comm.IChannelInterfaceFactory;
+import org.apache.hyracks.api.comm.ICloseableBufferAcceptor;
+import org.apache.hyracks.api.comm.NetworkAddress;
+import org.apache.hyracks.api.exceptions.NetException;
+import org.apache.hyracks.control.nc.NodeControllerService;
+import org.apache.hyracks.net.protocols.muxdemux.ChannelControlBlock;
+import org.apache.hyracks.net.protocols.muxdemux.IChannelOpenListener;
+import org.apache.hyracks.net.protocols.muxdemux.MultiplexedConnection;
+import org.apache.hyracks.net.protocols.muxdemux.MuxDemux;
+import org.apache.hyracks.net.protocols.muxdemux.MuxDemuxPerformanceCounters;
+
+public class MessagingNetworkManager {
+
+    private static final Logger LOGGER = Logger.getLogger(MessagingNetworkManager.class.getName());
+    private static final int MAX_CONNECTION_ATTEMPTS = 5;
+    private final MuxDemux md;
+    private NetworkAddress localNetworkAddress;
+    private NetworkAddress publicNetworkAddress;
+    private final Map<String, IChannelControlBlock> ncChannels = new HashMap<>();
+    private final NodeControllerService ncs;
+    private final Map<IChannelControlBlock, ICloseableBufferAcceptor> channelFullBufferAcceptor = new HashMap<>();
+
+    public MessagingNetworkManager(NodeControllerService ncs, String inetAddress, int inetPort, int nThreads,
+            String publicInetAddress, int publicInetPort, IChannelInterfaceFactory channelInterfaceFactory) {
+        this.ncs = ncs;
+        md = new MuxDemux(new InetSocketAddress(inetAddress, inetPort), new ChannelOpenListener(), nThreads,
+                MAX_CONNECTION_ATTEMPTS, channelInterfaceFactory);
+        publicNetworkAddress = new NetworkAddress(publicInetAddress, publicInetPort);
+    }
+
+    public void start() throws IOException {
+        md.start();
+        InetSocketAddress sockAddr = md.getLocalAddress();
+        localNetworkAddress = new NetworkAddress(sockAddr.getHostString(), sockAddr.getPort());
+
+        // See if the public address was explicitly specified, and if not,
+        // make it a copy of localNetworkAddress
+        if (publicNetworkAddress.getAddress() == null) {
+            publicNetworkAddress = localNetworkAddress;
+        } else {
+            // Likewise for public port
+            if (publicNetworkAddress.getPort() == 0) {
+                publicNetworkAddress = new NetworkAddress(publicNetworkAddress.getAddress(), sockAddr.getPort());
+            }
+        }
+    }
+
+    public void stop() {
+        // Currently there is nothing to stop
+    }
+
+    public IChannelControlBlock getMessagingChannel(String nodeId) throws Exception {
+        synchronized (ncChannels) {
+            IChannelControlBlock ccb = ncChannels.get(nodeId);
+            if (ccb == null) {
+                // Establish new connection
+                ccb = establishNewConnection(nodeId);
+                addOpenChannel(nodeId, ccb);
+            }
+            return ccb;
+        }
+    }
+
+    private ChannelControlBlock connect(SocketAddress remoteAddress) throws InterruptedException, NetException {
+        MultiplexedConnection mConn = md.connect((InetSocketAddress) remoteAddress);
+        return mConn.openChannel();
+    }
+
+    public MuxDemuxPerformanceCounters getPerformanceCounters() {
+        return md.getPerformanceCounters();
+    }
+
+    public NetworkAddress getPublicNetworkAddress() {
+        return publicNetworkAddress;
+    }
+
+    private void prepareMessagingInitialMessage(String ncId, final ByteBuffer buffer) throws NetException {
+        /*
+         * The messaging initial message contains the node id of the node
+         * which requested the channel to be opened.
+         */
+        int intialMsgLength = Integer.BYTES + ncId.length();
+        if (intialMsgLength > buffer.capacity()) {
+            throw new NetException("Initial message exceded the channel buffer size " + buffer.capacity() + " bytes");
+        }
+        buffer.clear();
+        buffer.putInt(ncId.length());
+        buffer.put(ncId.getBytes());
+        buffer.flip();
+    }
+
+    private IChannelControlBlock establishNewConnection(String nodeId) throws Exception {
+        Map<String, NodeControllerInfo> nodeControllers = ncs.getNodeControllersInfo();
+
+        // Get the node messaging address from its info
+        NodeControllerInfo nodeControllerInfo = nodeControllers.get(nodeId);
+        if (nodeControllerInfo == null) {
+            throw new NetException("Could not find node: " + nodeId);
+        }
+        NetworkAddress nodeMessagingNeAddress = nodeControllerInfo.getMessagingNetworkAddress();
+        SocketAddress nodeAddress = new InetSocketAddress(InetAddress.getByName(nodeMessagingNeAddress.getAddress()),
+                nodeMessagingNeAddress.getPort());
+
+        // Open the channel
+        IChannelControlBlock ccb = connect(nodeAddress);
+        try {
+            // Prepare the initial message buffer
+            ByteBuffer initialBuffer = ccb.getReadInterface().getBufferFactory().createBuffer();
+            prepareMessagingInitialMessage(ncs.getId(), initialBuffer);
+            // Send the initial messaging channel handshake message to register the opened channel on both nodes
+            ccb.getWriteInterface().getFullBufferAcceptor().accept(initialBuffer);
+            return ccb;
+        } catch (NetException e) {
+            closeChannel(ccb);
+            throw e;
+        }
+    }
+
+    private void addOpenChannel(final String nodeId, final IChannelControlBlock ccb) {
+        synchronized (ncChannels) {
+            if (ncChannels.get(nodeId) == null) {
+                ncChannels.put(nodeId, ccb);
+            } else {
+                closeChannel(ccb);
+                /*
+                 * TODO Currently there is a chance that two nodes will open
+                 * a channel to each other at the exact same time and both will
+                 * end up using a half closed channel. While this isn't a big issue,
+                 * it should be eliminated by introducing negotiation protocol
+                 * between nodes to decide which channel to use and which channel
+                 * to close fully.
+                 */
+            }
+        }
+    }
+
+    private void closeChannel(IChannelControlBlock ccb) {
+        ccb.getWriteInterface().getFullBufferAcceptor().close();
+    }
+
+    private class ChannelOpenListener implements IChannelOpenListener {
+        @Override
+        public void channelOpened(ChannelControlBlock channel) {
+            // Store the channel's original acceptor (which is set by the application)
+            ICloseableBufferAcceptor fullBufferAcceptor = channel.getReadInterface().getFullBufferAcceptor();
+            synchronized (channelFullBufferAcceptor) {
+                channelFullBufferAcceptor.put(channel, fullBufferAcceptor);
+            }
+            // Temporary set the acceptor to InitialBufferAcceptor to read the initial message
+            channel.getReadInterface().setFullBufferAcceptor(new InitialBufferAcceptor(channel));
+        }
+    }
+
+    private class InitialBufferAcceptor implements ICloseableBufferAcceptor {
+        private final ChannelControlBlock ccb;
+
+        public InitialBufferAcceptor(ChannelControlBlock ccb) {
+            this.ccb = ccb;
+        }
+
+        @Override
+        public void accept(ByteBuffer buffer) {
+            String nodeId = readMessagingInitialMessage(buffer);
+            if (LOGGER.isLoggable(Level.FINE)) {
+                LOGGER.fine("Opened messaging channel with node: " + nodeId);
+            }
+            // Return the channel's original acceptor
+            ICloseableBufferAcceptor originalAcceptor;
+            synchronized (channelFullBufferAcceptor) {
+                originalAcceptor = channelFullBufferAcceptor.remove(ccb);
+                if (originalAcceptor == null) {
+                    throw new IllegalStateException("Could not find channel acceptor");
+                }
+            }
+            ccb.getReadInterface().setFullBufferAcceptor(originalAcceptor);
+            addOpenChannel(nodeId, ccb);
+        }
+
+        @Override
+        public void close() {
+            // Nothing to close
+        }
+
+        @Override
+        public void error(int ecode) {
+            // Errors should be handled in the application
+        }
+
+        private String readMessagingInitialMessage(ByteBuffer buffer) {
+            int nodeIdLength = buffer.getInt();
+            byte[] stringBytes = new byte[nodeIdLength];
+            buffer.get(stringBytes);
+            return new String(stringBytes);
+        }
+    }
+}
\ No newline at end of file
diff --git a/hyracks-fullstack/hyracks/hyracks-control/hyracks-control-nc/src/main/java/org/apache/hyracks/control/nc/net/NetworkManager.java b/hyracks-fullstack/hyracks/hyracks-control/hyracks-control-nc/src/main/java/org/apache/hyracks/control/nc/net/NetworkManager.java
index 4b28aef..325966a 100644
--- a/hyracks-fullstack/hyracks/hyracks-control/hyracks-control-nc/src/main/java/org/apache/hyracks/control/nc/net/NetworkManager.java
+++ b/hyracks-fullstack/hyracks/hyracks-control/hyracks-control-nc/src/main/java/org/apache/hyracks/control/nc/net/NetworkManager.java
@@ -25,16 +25,17 @@
 import java.util.logging.Level;
 import java.util.logging.Logger;
 
+import org.apache.hyracks.api.comm.IChannelInterfaceFactory;
+import org.apache.hyracks.api.comm.ICloseableBufferAcceptor;
 import org.apache.hyracks.api.comm.NetworkAddress;
 import org.apache.hyracks.api.dataflow.ConnectorDescriptorId;
 import org.apache.hyracks.api.exceptions.HyracksException;
+import org.apache.hyracks.api.exceptions.NetException;
 import org.apache.hyracks.api.job.JobId;
 import org.apache.hyracks.api.partitions.PartitionId;
 import org.apache.hyracks.comm.channels.IChannelConnectionFactory;
 import org.apache.hyracks.comm.channels.NetworkOutputChannel;
 import org.apache.hyracks.control.nc.partitions.PartitionManager;
-import org.apache.hyracks.net.buffers.ICloseableBufferAcceptor;
-import org.apache.hyracks.net.exceptions.NetException;
 import org.apache.hyracks.net.protocols.muxdemux.ChannelControlBlock;
 import org.apache.hyracks.net.protocols.muxdemux.IChannelOpenListener;
 import org.apache.hyracks.net.protocols.muxdemux.MultiplexedConnection;
@@ -58,13 +59,13 @@
 
     private NetworkAddress publicNetworkAddress;
 
-    public NetworkManager(String inetAddress, int inetPort, PartitionManager partitionManager, int nThreads, int nBuffers,
-                          String publicInetAddress, int publicInetPort)
-            throws IOException {
+    public NetworkManager(String inetAddress, int inetPort, PartitionManager partitionManager, int nThreads,
+            int nBuffers, String publicInetAddress, int publicInetPort,
+            IChannelInterfaceFactory channelInterfaceFactory) {
         this.partitionManager = partitionManager;
         this.nBuffers = nBuffers;
         md = new MuxDemux(new InetSocketAddress(inetAddress, inetPort), new ChannelOpenListener(), nThreads,
-                MAX_CONNECTION_ATTEMPTS);
+                MAX_CONNECTION_ATTEMPTS, channelInterfaceFactory);
         // Just save these values for the moment; may be reset in start()
         publicNetworkAddress = new NetworkAddress(publicInetAddress, publicInetPort);
     }
@@ -78,12 +79,10 @@
         // make it a copy of localNetworkAddress
         if (publicNetworkAddress.getAddress() == null) {
             publicNetworkAddress = localNetworkAddress;
-        }
-        else {
+        } else {
             // Likewise for public port
             if (publicNetworkAddress.getPort() == 0) {
-                publicNetworkAddress = new NetworkAddress
-                    (publicNetworkAddress.getAddress(), sockAddr.getPort());
+                publicNetworkAddress = new NetworkAddress(publicNetworkAddress.getAddress(), sockAddr.getPort());
             }
         }
     }
@@ -100,6 +99,7 @@
 
     }
 
+    @Override
     public ChannelControlBlock connect(SocketAddress remoteAddress) throws InterruptedException, NetException {
         MultiplexedConnection mConn = md.connect((InetSocketAddress) remoteAddress);
         return mConn.openChannel();
diff --git a/hyracks-fullstack/hyracks/hyracks-hdfs/hyracks-hdfs-core/src/test/java/org/apache/hyracks/hdfs/scheduler/SchedulerTest.java b/hyracks-fullstack/hyracks/hyracks-hdfs/hyracks-hdfs-core/src/test/java/org/apache/hyracks/hdfs/scheduler/SchedulerTest.java
index 373fe21..13cb171 100644
--- a/hyracks-fullstack/hyracks/hyracks-hdfs/hyracks-hdfs-core/src/test/java/org/apache/hyracks/hdfs/scheduler/SchedulerTest.java
+++ b/hyracks-fullstack/hyracks/hyracks-hdfs/hyracks-hdfs-core/src/test/java/org/apache/hyracks/hdfs/scheduler/SchedulerTest.java
@@ -21,24 +21,22 @@
 
 import java.io.FileReader;
 import java.io.IOException;
-import java.net.InetAddress;
-import java.util.HashMap;
 import java.util.Map;
-
-import junit.framework.Assert;
-import junit.framework.TestCase;
 
 import org.apache.hadoop.fs.Path;
 import org.apache.hadoop.mapred.FileSplit;
 import org.apache.hadoop.mapred.InputSplit;
-import org.xml.sax.InputSource;
-import org.xml.sax.SAXException;
-
 import org.apache.hyracks.api.client.NodeControllerInfo;
 import org.apache.hyracks.api.client.NodeStatus;
 import org.apache.hyracks.api.comm.NetworkAddress;
 import org.apache.hyracks.api.topology.ClusterTopology;
 import org.apache.hyracks.api.topology.TopologyDefinitionParser;
+import org.apache.hyracks.hdfs.utils.TestUtils;
+import org.xml.sax.InputSource;
+import org.xml.sax.SAXException;
+
+import junit.framework.Assert;
+import junit.framework.TestCase;
 
 @SuppressWarnings("deprecation")
 public class SchedulerTest extends TestCase {
@@ -60,13 +58,8 @@
      * @throws Exception
      */
     public void testSchedulerSimple() throws Exception {
-        Map<String, NodeControllerInfo> ncNameToNcInfos = new HashMap<String, NodeControllerInfo>();
-        ncNameToNcInfos.put("nc1", new NodeControllerInfo("nc1", NodeStatus.ALIVE, new NetworkAddress("10.0.0.1", 5099), new NetworkAddress("10.0.0.1", 5098)));
-        ncNameToNcInfos.put("nc2", new NodeControllerInfo("nc2", NodeStatus.ALIVE, new NetworkAddress("10.0.0.2", 5099), new NetworkAddress("10.0.0.2", 5098)));
-        ncNameToNcInfos.put("nc3", new NodeControllerInfo("nc3", NodeStatus.ALIVE, new NetworkAddress("10.0.0.3", 5099), new NetworkAddress("10.0.0.3", 5098)));
-        ncNameToNcInfos.put("nc4", new NodeControllerInfo("nc4", NodeStatus.ALIVE, new NetworkAddress("10.0.0.4", 5099), new NetworkAddress("10.0.0.4", 5098)));
-        ncNameToNcInfos.put("nc5", new NodeControllerInfo("nc5", NodeStatus.ALIVE, new NetworkAddress("10.0.0.5", 5099), new NetworkAddress("10.0.0.5", 5098)));
-        ncNameToNcInfos.put("nc6", new NodeControllerInfo("nc6", NodeStatus.ALIVE, new NetworkAddress("10.0.0.6", 5099), new NetworkAddress("10.0.0.6", 5098)));
+        Map<String, NodeControllerInfo> ncNameToNcInfos = TestUtils.generateNodeControllerInfo(6, "nc", "10.0.0.", 5099,
+                5098, 5097);
 
         InputSplit[] fileSplits = new InputSplit[6];
         fileSplits[0] = new FileSplit(new Path("part-1"), 0, 0, new String[] { "10.0.0.1", "10.0.0.2", "10.0.0.3" });
@@ -98,13 +91,17 @@
      * @throws Exception
      */
     public void testSchedulerLargerHDFS() throws Exception {
-        Map<String, NodeControllerInfo> ncNameToNcInfos = new HashMap<String, NodeControllerInfo>();
-        ncNameToNcInfos.put("nc1", new NodeControllerInfo("nc1", NodeStatus.ALIVE, new NetworkAddress("10.0.0.1", 5099), new NetworkAddress("10.0.0.1", 5098)));
-        ncNameToNcInfos.put("nc2", new NodeControllerInfo("nc2", NodeStatus.ALIVE, new NetworkAddress("10.0.0.2", 5099), new NetworkAddress("10.0.0.2", 5098)));
-        ncNameToNcInfos.put("nc3", new NodeControllerInfo("nc3", NodeStatus.ALIVE, new NetworkAddress("10.0.0.3", 5099), new NetworkAddress("10.0.0.3", 5098)));
-        ncNameToNcInfos.put("nc4", new NodeControllerInfo("nc4", NodeStatus.ALIVE, new NetworkAddress("10.0.0.4", 5099), new NetworkAddress("10.0.0.4", 5098)));
-        ncNameToNcInfos.put("nc7", new NodeControllerInfo("nc7", NodeStatus.ALIVE, new NetworkAddress("10.0.0.7", 5099), new NetworkAddress("10.0.0.5", 5098)));
-        ncNameToNcInfos.put("nc12", new NodeControllerInfo("nc12", NodeStatus.ALIVE, new NetworkAddress("10.0.0.12", 5099), new NetworkAddress("10.0.0.5", 5098)));
+        int dataPort = 5099;
+        int resultPort = 5098;
+        int messagingPort = 5097;
+        Map<String, NodeControllerInfo> ncNameToNcInfos = TestUtils.generateNodeControllerInfo(4, "nc", "10.0.0.",
+                dataPort, resultPort, messagingPort);
+        ncNameToNcInfos.put("nc7",
+                new NodeControllerInfo("nc7", NodeStatus.ALIVE, new NetworkAddress("10.0.0.7", dataPort),
+                        new NetworkAddress("10.0.0.5", resultPort), new NetworkAddress("10.0.0.5", messagingPort)));
+        ncNameToNcInfos.put("nc12",
+                new NodeControllerInfo("nc12", NodeStatus.ALIVE, new NetworkAddress("10.0.0.12", dataPort),
+                        new NetworkAddress("10.0.0.5", resultPort), new NetworkAddress("10.0.0.5", messagingPort)));
 
         InputSplit[] fileSplits = new InputSplit[12];
         fileSplits[0] = new FileSplit(new Path("part-1"), 0, 0, new String[] { "10.0.0.1", "10.0.0.2", "10.0.0.3" });
@@ -115,7 +112,8 @@
         fileSplits[5] = new FileSplit(new Path("part-6"), 0, 0, new String[] { "10.0.0.2", "10.0.0.3", "10.0.0.5" });
         fileSplits[6] = new FileSplit(new Path("part-7"), 0, 0, new String[] { "10.0.0.1", "10.0.0.2", "10.0.0.3" });
         fileSplits[7] = new FileSplit(new Path("part-8"), 0, 0, new String[] { "10.0.0.3", "10.0.0.4", "10.0.0.5" });
-        fileSplits[8] = new FileSplit(new Path("part-12"), 0, 0, new String[] { "10.0.0.14", "10.0.0.11", "10.0.0.13" });
+        fileSplits[8] = new FileSplit(new Path("part-12"), 0, 0,
+                new String[] { "10.0.0.14", "10.0.0.11", "10.0.0.13" });
         fileSplits[9] = new FileSplit(new Path("part-10"), 0, 0, new String[] { "10.0.0.2", "10.0.0.1", "10.0.0.6" });
         fileSplits[10] = new FileSplit(new Path("part-11"), 0, 0, new String[] { "10.0.0.3", "10.0.0.4", "10.0.0.7" });
         fileSplits[11] = new FileSplit(new Path("part-9"), 0, 0, new String[] { "10.0.0.4", "10.0.0.5", "10.0.0.6" });
@@ -123,8 +121,8 @@
         Scheduler scheduler = new Scheduler(ncNameToNcInfos);
         String[] locationConstraints = scheduler.getLocationConstraints(fileSplits);
 
-        String[] expectedResults = new String[] { "nc1", "nc4", "nc4", "nc1", "nc3", "nc2", "nc2", "nc3", "nc12",
-                "nc7", "nc7", "nc12" };
+        String[] expectedResults = new String[] { "nc1", "nc4", "nc4", "nc1", "nc3", "nc2", "nc2", "nc3", "nc12", "nc7",
+                "nc7", "nc12" };
         for (int i = 0; i < locationConstraints.length; i++) {
             Assert.assertEquals(locationConstraints[i], expectedResults[i]);
         }
@@ -145,13 +143,8 @@
      * @throws Exception
      */
     public void testSchedulerSmallerHDFS() throws Exception {
-        Map<String, NodeControllerInfo> ncNameToNcInfos = new HashMap<String, NodeControllerInfo>();
-        ncNameToNcInfos.put("nc1", new NodeControllerInfo("nc1", NodeStatus.ALIVE, new NetworkAddress("10.0.0.1", 5099), new NetworkAddress("10.0.0.1", 5098)));
-        ncNameToNcInfos.put("nc2", new NodeControllerInfo("nc2", NodeStatus.ALIVE, new NetworkAddress("10.0.0.2", 5099), new NetworkAddress("10.0.0.2", 5098)));
-        ncNameToNcInfos.put("nc3", new NodeControllerInfo("nc3", NodeStatus.ALIVE, new NetworkAddress("10.0.0.3", 5099), new NetworkAddress("10.0.0.3", 5098)));
-        ncNameToNcInfos.put("nc4", new NodeControllerInfo("nc4", NodeStatus.ALIVE, new NetworkAddress("10.0.0.4", 5099), new NetworkAddress("10.0.0.4", 5098)));
-        ncNameToNcInfos.put("nc5", new NodeControllerInfo("nc5", NodeStatus.ALIVE, new NetworkAddress("10.0.0.5", 5099), new NetworkAddress("10.0.0.5", 5098)));
-        ncNameToNcInfos.put("nc6", new NodeControllerInfo("nc6", NodeStatus.ALIVE, new NetworkAddress("10.0.0.6", 5099), new NetworkAddress("10.0.0.6", 5098)));
+        Map<String, NodeControllerInfo> ncNameToNcInfos = TestUtils.generateNodeControllerInfo(6, "nc", "10.0.0.", 5099,
+                5098, 5097);
 
         InputSplit[] fileSplits = new InputSplit[12];
         fileSplits[0] = new FileSplit(new Path("part-1"), 0, 0, new String[] { "10.0.0.1", "10.0.0.2", "10.0.0.3" });
@@ -191,13 +184,8 @@
      * @throws Exception
      */
     public void testSchedulerSmallerHDFSOdd() throws Exception {
-        Map<String, NodeControllerInfo> ncNameToNcInfos = new HashMap<String, NodeControllerInfo>();
-        ncNameToNcInfos.put("nc1", new NodeControllerInfo("nc1", NodeStatus.ALIVE, new NetworkAddress("10.0.0.1", 5099), new NetworkAddress("10.0.0.1", 5098)));
-        ncNameToNcInfos.put("nc2", new NodeControllerInfo("nc2", NodeStatus.ALIVE, new NetworkAddress("10.0.0.2", 5099), new NetworkAddress("10.0.0.2", 5098)));
-        ncNameToNcInfos.put("nc3", new NodeControllerInfo("nc3", NodeStatus.ALIVE, new NetworkAddress("10.0.0.3", 5099), new NetworkAddress("10.0.0.3", 5098)));
-        ncNameToNcInfos.put("nc4", new NodeControllerInfo("nc4", NodeStatus.ALIVE, new NetworkAddress("10.0.0.4", 5099), new NetworkAddress("10.0.0.4", 5098)));
-        ncNameToNcInfos.put("nc5", new NodeControllerInfo("nc5", NodeStatus.ALIVE, new NetworkAddress("10.0.0.5", 5099), new NetworkAddress("10.0.0.5", 5098)));
-        ncNameToNcInfos.put("nc6", new NodeControllerInfo("nc6", NodeStatus.ALIVE, new NetworkAddress("10.0.0.6", 5099), new NetworkAddress("10.0.0.6", 5098)));
+        Map<String, NodeControllerInfo> ncNameToNcInfos = TestUtils.generateNodeControllerInfo(6, "nc", "10.0.0.", 5099,
+                5098, 5097);
 
         InputSplit[] fileSplits = new InputSplit[13];
         fileSplits[0] = new FileSplit(new Path("part-1"), 0, 0, new String[] { "10.0.0.1", "10.0.0.2", "10.0.0.3" });
@@ -239,13 +227,8 @@
      * @throws Exception
      */
     public void testSchedulercBoundary() throws Exception {
-        Map<String, NodeControllerInfo> ncNameToNcInfos = new HashMap<String, NodeControllerInfo>();
-        ncNameToNcInfos.put("nc1", new NodeControllerInfo("nc1", NodeStatus.ALIVE, new NetworkAddress("10.0.0.1", 5099), new NetworkAddress("10.0.0.1", 5098)));
-        ncNameToNcInfos.put("nc2", new NodeControllerInfo("nc2", NodeStatus.ALIVE, new NetworkAddress("10.0.0.2", 5099), new NetworkAddress("10.0.0.2", 5098)));
-        ncNameToNcInfos.put("nc3", new NodeControllerInfo("nc3", NodeStatus.ALIVE, new NetworkAddress("10.0.0.3", 5099), new NetworkAddress("10.0.0.3", 5098)));
-        ncNameToNcInfos.put("nc4", new NodeControllerInfo("nc4", NodeStatus.ALIVE, new NetworkAddress("10.0.0.4", 5099), new NetworkAddress("10.0.0.4", 5098)));
-        ncNameToNcInfos.put("nc5", new NodeControllerInfo("nc5", NodeStatus.ALIVE, new NetworkAddress("10.0.0.5", 5099), new NetworkAddress("10.0.0.5", 5098)));
-        ncNameToNcInfos.put("nc6", new NodeControllerInfo("nc6", NodeStatus.ALIVE, new NetworkAddress("10.0.0.6", 5099), new NetworkAddress("10.0.0.6", 5098)));
+        Map<String, NodeControllerInfo> ncNameToNcInfos = TestUtils.generateNodeControllerInfo(6, "nc", "10.0.0.", 5099,
+                5098, 5097);
 
         /** test empty file splits */
         InputSplit[] fileSplits = new InputSplit[0];
diff --git a/hyracks-fullstack/hyracks/hyracks-hdfs/hyracks-hdfs-core/src/test/java/org/apache/hyracks/hdfs/utils/TestUtils.java b/hyracks-fullstack/hyracks/hyracks-hdfs/hyracks-hdfs-core/src/test/java/org/apache/hyracks/hdfs/utils/TestUtils.java
index c2ad4a0..8755cf3 100644
--- a/hyracks-fullstack/hyracks/hyracks-hdfs/hyracks-hdfs-core/src/test/java/org/apache/hyracks/hdfs/utils/TestUtils.java
+++ b/hyracks-fullstack/hyracks/hyracks-hdfs/hyracks-hdfs-core/src/test/java/org/apache/hyracks/hdfs/utils/TestUtils.java
@@ -22,6 +22,12 @@
 import java.io.BufferedReader;
 import java.io.File;
 import java.io.FileReader;
+import java.util.HashMap;
+import java.util.Map;
+
+import org.apache.hyracks.api.client.NodeControllerInfo;
+import org.apache.hyracks.api.client.NodeStatus;
+import org.apache.hyracks.api.comm.NetworkAddress;
 
 public class TestUtils {
 
@@ -38,8 +44,8 @@
                     throw new Exception("Actual result changed at line " + num + ":\n< " + lineExpected + "\n> ");
                 }
                 if (!equalStrings(lineExpected, lineActual)) {
-                    throw new Exception("Result for changed at line " + num + ":\n< " + lineExpected + "\n> "
-                            + lineActual);
+                    throw new Exception(
+                            "Result for changed at line " + num + ":\n< " + lineExpected + "\n> " + lineActual);
                 }
                 ++num;
             }
@@ -94,4 +100,16 @@
         return true;
     }
 
+    public static Map<String, NodeControllerInfo> generateNodeControllerInfo(int numberOfNodes, String ncNamePrefix,
+            String addressPrefix, int netPort, int dataPort, int messagingPort) {
+        Map<String, NodeControllerInfo> ncNameToNcInfos = new HashMap<String, NodeControllerInfo>();
+        for (int i = 1; i <= numberOfNodes; i++) {
+            String ncId = ncNamePrefix + i;
+            String ncAddress = addressPrefix + i;
+            ncNameToNcInfos.put(ncId,
+                    new NodeControllerInfo(ncId, NodeStatus.ALIVE, new NetworkAddress(ncAddress, netPort),
+                            new NetworkAddress(ncAddress, dataPort), new NetworkAddress(ncAddress, messagingPort)));
+        }
+        return ncNameToNcInfos;
+    }
 }
diff --git a/hyracks-fullstack/hyracks/hyracks-hdfs/hyracks-hdfs-core/src/test/java/org/apache/hyracks/hdfs2/scheduler/SchedulerTest.java b/hyracks-fullstack/hyracks/hyracks-hdfs/hyracks-hdfs-core/src/test/java/org/apache/hyracks/hdfs2/scheduler/SchedulerTest.java
index 6eabb71..793e029 100644
--- a/hyracks-fullstack/hyracks/hyracks-hdfs/hyracks-hdfs-core/src/test/java/org/apache/hyracks/hdfs2/scheduler/SchedulerTest.java
+++ b/hyracks-fullstack/hyracks/hyracks-hdfs/hyracks-hdfs-core/src/test/java/org/apache/hyracks/hdfs2/scheduler/SchedulerTest.java
@@ -19,22 +19,18 @@
 
 package org.apache.hyracks.hdfs2.scheduler;
 
-import java.net.InetAddress;
 import java.util.ArrayList;
-import java.util.HashMap;
 import java.util.List;
 import java.util.Map;
-
-import junit.framework.Assert;
-import junit.framework.TestCase;
 
 import org.apache.hadoop.fs.Path;
 import org.apache.hadoop.mapreduce.InputSplit;
 import org.apache.hadoop.mapreduce.lib.input.FileSplit;
-
 import org.apache.hyracks.api.client.NodeControllerInfo;
-import org.apache.hyracks.api.client.NodeStatus;
-import org.apache.hyracks.api.comm.NetworkAddress;
+import org.apache.hyracks.hdfs.utils.TestUtils;
+
+import junit.framework.Assert;
+import junit.framework.TestCase;
 
 /**
  * Test case for the new HDFS API scheduler
@@ -47,13 +43,8 @@
      * @throws Exception
      */
     public void testSchedulerSimple() throws Exception {
-        Map<String, NodeControllerInfo> ncNameToNcInfos = new HashMap<String, NodeControllerInfo>();
-        ncNameToNcInfos.put("nc1", new NodeControllerInfo("nc1", NodeStatus.ALIVE, new NetworkAddress("10.0.0.1", 5099), new NetworkAddress("10.0.0.1", 5098)));
-        ncNameToNcInfos.put("nc2", new NodeControllerInfo("nc2", NodeStatus.ALIVE, new NetworkAddress("10.0.0.2", 5099), new NetworkAddress("10.0.0.2", 5098)));
-        ncNameToNcInfos.put("nc3", new NodeControllerInfo("nc3", NodeStatus.ALIVE, new NetworkAddress("10.0.0.3", 5099), new NetworkAddress("10.0.0.3", 5098)));
-        ncNameToNcInfos.put("nc4", new NodeControllerInfo("nc4", NodeStatus.ALIVE, new NetworkAddress("10.0.0.4", 5099), new NetworkAddress("10.0.0.4", 5098)));
-        ncNameToNcInfos.put("nc5", new NodeControllerInfo("nc5", NodeStatus.ALIVE, new NetworkAddress("10.0.0.5", 5099), new NetworkAddress("10.0.0.5", 5098)));
-        ncNameToNcInfos.put("nc6", new NodeControllerInfo("nc6", NodeStatus.ALIVE, new NetworkAddress("10.0.0.6", 5099), new NetworkAddress("10.0.0.6", 5098)));
+        Map<String, NodeControllerInfo> ncNameToNcInfos = TestUtils.generateNodeControllerInfo(6, "nc", "10.0.0.", 5099,
+                5098, 5097);
 
         List<InputSplit> fileSplits = new ArrayList<InputSplit>();
         fileSplits.add(new FileSplit(new Path("part-1"), 0, 0, new String[] { "10.0.0.1", "10.0.0.2", "10.0.0.3" }));
@@ -79,13 +70,8 @@
      * @throws Exception
      */
     public void testSchedulerLargerHDFS() throws Exception {
-        Map<String, NodeControllerInfo> ncNameToNcInfos = new HashMap<String, NodeControllerInfo>();
-        ncNameToNcInfos.put("nc1", new NodeControllerInfo("nc1", NodeStatus.ALIVE, new NetworkAddress("10.0.0.1", 5099), new NetworkAddress("10.0.0.1", 5098)));
-        ncNameToNcInfos.put("nc2", new NodeControllerInfo("nc2", NodeStatus.ALIVE, new NetworkAddress("10.0.0.2", 5099), new NetworkAddress("10.0.0.2", 5098)));
-        ncNameToNcInfos.put("nc3", new NodeControllerInfo("nc3", NodeStatus.ALIVE, new NetworkAddress("10.0.0.3", 5099), new NetworkAddress("10.0.0.3", 5098)));
-        ncNameToNcInfos.put("nc4", new NodeControllerInfo("nc4", NodeStatus.ALIVE, new NetworkAddress("10.0.0.4", 5099), new NetworkAddress("10.0.0.4", 5098)));
-        ncNameToNcInfos.put("nc5", new NodeControllerInfo("nc5", NodeStatus.ALIVE, new NetworkAddress("10.0.0.5", 5099), new NetworkAddress("10.0.0.5", 5098)));
-        ncNameToNcInfos.put("nc6", new NodeControllerInfo("nc6", NodeStatus.ALIVE, new NetworkAddress("10.0.0.6", 5099), new NetworkAddress("10.0.0.6", 5098)));
+        Map<String, NodeControllerInfo> ncNameToNcInfos = TestUtils.generateNodeControllerInfo(6, "nc", "10.0.0.", 5099,
+                5098, 5097);
 
         List<InputSplit> fileSplits = new ArrayList<InputSplit>();
         fileSplits.add(new FileSplit(new Path("part-1"), 0, 0, new String[] { "10.0.0.1", "10.0.0.2", "10.0.0.3" }));
@@ -118,13 +104,8 @@
      * @throws Exception
      */
     public void testSchedulerSmallerHDFS() throws Exception {
-        Map<String, NodeControllerInfo> ncNameToNcInfos = new HashMap<String, NodeControllerInfo>();
-        ncNameToNcInfos.put("nc1", new NodeControllerInfo("nc1", NodeStatus.ALIVE, new NetworkAddress("10.0.0.1", 5099), new NetworkAddress("10.0.0.1", 5098)));
-        ncNameToNcInfos.put("nc2", new NodeControllerInfo("nc2", NodeStatus.ALIVE, new NetworkAddress("10.0.0.2", 5099), new NetworkAddress("10.0.0.2", 5098)));
-        ncNameToNcInfos.put("nc3", new NodeControllerInfo("nc3", NodeStatus.ALIVE, new NetworkAddress("10.0.0.3", 5099), new NetworkAddress("10.0.0.3", 5098)));
-        ncNameToNcInfos.put("nc4", new NodeControllerInfo("nc4", NodeStatus.ALIVE, new NetworkAddress("10.0.0.4", 5099), new NetworkAddress("10.0.0.4", 5098)));
-        ncNameToNcInfos.put("nc5", new NodeControllerInfo("nc5", NodeStatus.ALIVE, new NetworkAddress("10.0.0.5", 5099), new NetworkAddress("10.0.0.5", 5098)));
-        ncNameToNcInfos.put("nc6", new NodeControllerInfo("nc6", NodeStatus.ALIVE, new NetworkAddress("10.0.0.6", 5099), new NetworkAddress("10.0.0.6", 5098)));
+        Map<String, NodeControllerInfo> ncNameToNcInfos = TestUtils.generateNodeControllerInfo(6, "nc", "10.0.0.", 5099,
+                5098, 5097);
 
         List<InputSplit> fileSplits = new ArrayList<InputSplit>();
         fileSplits.add(new FileSplit(new Path("part-1"), 0, 0, new String[] { "10.0.0.1", "10.0.0.2", "10.0.0.3" }));
@@ -157,13 +138,8 @@
      * @throws Exception
      */
     public void testSchedulerSmallerHDFSOdd() throws Exception {
-        Map<String, NodeControllerInfo> ncNameToNcInfos = new HashMap<String, NodeControllerInfo>();
-        ncNameToNcInfos.put("nc1", new NodeControllerInfo("nc1", NodeStatus.ALIVE, new NetworkAddress("10.0.0.1", 5099), new NetworkAddress("10.0.0.1", 5098)));
-        ncNameToNcInfos.put("nc2", new NodeControllerInfo("nc2", NodeStatus.ALIVE, new NetworkAddress("10.0.0.2", 5099), new NetworkAddress("10.0.0.2", 5098)));
-        ncNameToNcInfos.put("nc3", new NodeControllerInfo("nc3", NodeStatus.ALIVE, new NetworkAddress("10.0.0.3", 5099), new NetworkAddress("10.0.0.3", 5098)));
-        ncNameToNcInfos.put("nc4", new NodeControllerInfo("nc4", NodeStatus.ALIVE, new NetworkAddress("10.0.0.4", 5099), new NetworkAddress("10.0.0.4", 5098)));
-        ncNameToNcInfos.put("nc5", new NodeControllerInfo("nc5", NodeStatus.ALIVE, new NetworkAddress("10.0.0.5", 5099), new NetworkAddress("10.0.0.5", 5098)));
-        ncNameToNcInfos.put("nc6", new NodeControllerInfo("nc6", NodeStatus.ALIVE, new NetworkAddress("10.0.0.6", 5099), new NetworkAddress("10.0.0.6", 5098)));
+        Map<String, NodeControllerInfo> ncNameToNcInfos = TestUtils.generateNodeControllerInfo(6, "nc", "10.0.0.", 5099,
+                5098, 5097);
 
         List<InputSplit> fileSplits = new ArrayList<InputSplit>();
         fileSplits.add(new FileSplit(new Path("part-1"), 0, 0, new String[] { "10.0.0.1", "10.0.0.2", "10.0.0.3" }));
diff --git a/hyracks-fullstack/hyracks/hyracks-net/pom.xml b/hyracks-fullstack/hyracks/hyracks-net/pom.xml
index a41b7e5..46494c3 100644
--- a/hyracks-fullstack/hyracks/hyracks-net/pom.xml
+++ b/hyracks-fullstack/hyracks/hyracks-net/pom.xml
@@ -42,6 +42,11 @@
 
   <dependencies>
     <dependency>
+      <groupId>org.apache.hyracks</groupId>
+      <artifactId>hyracks-api</artifactId>
+      <version>0.2.18-SNAPSHOT</version>
+    </dependency>
+    <dependency>
       <groupId>junit</groupId>
       <artifactId>junit</artifactId>
       <scope>test</scope>
diff --git a/hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/protocols/muxdemux/AbstractChannelReadInterface.java b/hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/protocols/muxdemux/AbstractChannelReadInterface.java
new file mode 100644
index 0000000..ff8d451
--- /dev/null
+++ b/hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/protocols/muxdemux/AbstractChannelReadInterface.java
@@ -0,0 +1,83 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements.  See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership.  The ASF licenses this file
+ * to you 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.apache.hyracks.net.protocols.muxdemux;
+
+import java.nio.ByteBuffer;
+
+import org.apache.hyracks.api.comm.IBufferAcceptor;
+import org.apache.hyracks.api.comm.IBufferFactory;
+import org.apache.hyracks.api.comm.IChannelReadInterface;
+import org.apache.hyracks.api.comm.ICloseableBufferAcceptor;
+
+public abstract class AbstractChannelReadInterface implements IChannelReadInterface {
+
+    protected ICloseableBufferAcceptor fba;
+    protected IBufferAcceptor emptyBufferAcceptor;
+    protected ByteBuffer currentReadBuffer;
+    protected IBufferFactory bufferFactory;
+    protected volatile int credits;
+
+    @Override
+    public void flush() {
+        if (currentReadBuffer != null) {
+            currentReadBuffer.flip();
+            fba.accept(currentReadBuffer);
+            currentReadBuffer = null;
+        }
+    }
+
+    public void reportError(int ecode) {
+        fba.error(ecode);
+    }
+
+    @Override
+    public void setFullBufferAcceptor(ICloseableBufferAcceptor fullBufferAcceptor) {
+        fba = fullBufferAcceptor;
+    }
+
+    @Override
+    public IBufferAcceptor getEmptyBufferAcceptor() {
+        return emptyBufferAcceptor;
+    }
+
+    @Override
+    public ICloseableBufferAcceptor getFullBufferAcceptor() {
+        return fba;
+    }
+
+    @Override
+    public int getCredits() {
+        return credits;
+    }
+
+    @Override
+    public void setReadCredits(int credits) {
+        this.credits = credits;
+    }
+
+    @Override
+    public IBufferFactory getBufferFactory() {
+        return bufferFactory;
+    }
+
+    @Override
+    public void setBufferFactory(IBufferFactory bufferFactory, int limit, int frameSize) {
+        this.bufferFactory = bufferFactory;
+    }
+}
diff --git a/hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/protocols/muxdemux/AbstractChannelWriteInterface.java b/hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/protocols/muxdemux/AbstractChannelWriteInterface.java
new file mode 100644
index 0000000..e50ffd2
--- /dev/null
+++ b/hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/protocols/muxdemux/AbstractChannelWriteInterface.java
@@ -0,0 +1,150 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements.  See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership.  The ASF licenses this file
+ * to you 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.apache.hyracks.net.protocols.muxdemux;
+
+import java.nio.ByteBuffer;
+import java.util.ArrayDeque;
+import java.util.Queue;
+import java.util.logging.Level;
+import java.util.logging.Logger;
+
+import org.apache.hyracks.api.comm.IBufferAcceptor;
+import org.apache.hyracks.api.comm.IChannelControlBlock;
+import org.apache.hyracks.api.comm.IChannelWriteInterface;
+import org.apache.hyracks.api.comm.ICloseableBufferAcceptor;
+
+public abstract class AbstractChannelWriteInterface implements IChannelWriteInterface {
+
+    private static final Logger LOGGER = Logger.getLogger(AbstractChannelWriteInterface.class.getName());
+    protected final IChannelControlBlock ccb;
+    protected final Queue<ByteBuffer> wiFullQueue;
+    protected boolean channelWritabilityState;
+    protected final int channelId;
+    protected IBufferAcceptor eba;
+    protected int credits;
+    protected boolean eos;
+    protected boolean eosSent;
+    protected int ecode;
+    protected boolean ecodeSent;
+    protected ByteBuffer currentWriteBuffer;
+    private final ICloseableBufferAcceptor fba;
+
+    public AbstractChannelWriteInterface(IChannelControlBlock ccb) {
+        this.ccb = ccb;
+        this.channelId = ccb.getChannelId();
+        wiFullQueue = new ArrayDeque<>();
+        fba = new CloseableBufferAcceptor();
+        credits = 0;
+        eos = false;
+        eosSent = false;
+        ecode = -1;
+        ecodeSent = false;
+    }
+
+    @Override
+    public void writeComplete() {
+        if (currentWriteBuffer.remaining() <= 0) {
+            currentWriteBuffer.clear();
+            eba.accept(currentWriteBuffer);
+            currentWriteBuffer = null;
+            adjustChannelWritability();
+        }
+    }
+
+    private boolean computeWritability() {
+        boolean writableDataPresent = currentWriteBuffer != null || !wiFullQueue.isEmpty();
+        if (writableDataPresent) {
+            return credits > 0;
+        }
+        if (eos && !eosSent) {
+            return true;
+        }
+        if (ecode >= 0 && !ecodeSent) {
+            return true;
+        }
+        return false;
+    }
+
+    @Override
+    public void adjustChannelWritability() {
+        boolean writable = computeWritability();
+        if (writable) {
+            if (!channelWritabilityState) {
+                ccb.markPendingWrite();
+            }
+        } else {
+            if (channelWritabilityState) {
+                ccb.unmarkPendingWrite();
+            }
+        }
+        channelWritabilityState = writable;
+    }
+
+    @Override
+    public void addCredits(int credit) {
+        credits += credit;
+    }
+
+    @Override
+    public void setEmptyBufferAcceptor(IBufferAcceptor emptyBufferAcceptor) {
+        eba = emptyBufferAcceptor;
+    }
+
+    @Override
+    public ICloseableBufferAcceptor getFullBufferAcceptor() {
+        return fba;
+    }
+
+    @Override
+    public int getCredits() {
+        return credits;
+    }
+
+    private class CloseableBufferAcceptor implements ICloseableBufferAcceptor {
+        @Override
+        public void accept(ByteBuffer buffer) {
+            synchronized (ccb) {
+                wiFullQueue.add(buffer);
+                adjustChannelWritability();
+            }
+        }
+
+        @Override
+        public void close() {
+            synchronized (ccb) {
+                if (eos) {
+                    if (LOGGER.isLoggable(Level.WARNING)) {
+                        LOGGER.warning("Received duplicate close() on channel: " + ccb.getChannelId());
+                    }
+                    return;
+                }
+                eos = true;
+                adjustChannelWritability();
+            }
+        }
+
+        @Override
+        public void error(int ecode) {
+            synchronized (ccb) {
+                AbstractChannelWriteInterface.this.ecode = ecode;
+                adjustChannelWritability();
+            }
+        }
+    }
+}
diff --git a/hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/protocols/muxdemux/ChannelControlBlock.java b/hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/protocols/muxdemux/ChannelControlBlock.java
index 6d82297..ba463d3 100644
--- a/hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/protocols/muxdemux/ChannelControlBlock.java
+++ b/hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/protocols/muxdemux/ChannelControlBlock.java
@@ -19,34 +19,29 @@
 package org.apache.hyracks.net.protocols.muxdemux;
 
 import java.io.IOException;
-import java.nio.ByteBuffer;
 import java.nio.channels.SocketChannel;
-import java.util.ArrayDeque;
-import java.util.Deque;
-import java.util.Queue;
 import java.util.concurrent.atomic.AtomicBoolean;
-import java.util.logging.Level;
-import java.util.logging.Logger;
 
-import org.apache.hyracks.net.buffers.IBufferAcceptor;
-import org.apache.hyracks.net.buffers.ICloseableBufferAcceptor;
-import org.apache.hyracks.net.exceptions.NetException;
+import org.apache.hyracks.api.comm.IChannelControlBlock;
+import org.apache.hyracks.api.comm.IChannelInterfaceFactory;
+import org.apache.hyracks.api.comm.IChannelReadInterface;
+import org.apache.hyracks.api.comm.IChannelWriteInterface;
+import org.apache.hyracks.api.exceptions.NetException;
+import org.apache.hyracks.net.protocols.muxdemux.MultiplexedConnection.WriterState;
 
 /**
  * Handle to a channel that represents a logical full-duplex communication end-point.
  *
  * @author vinayakb
  */
-public class ChannelControlBlock {
-    private static final Logger LOGGER = Logger.getLogger(ChannelControlBlock.class.getName());
-
+public class ChannelControlBlock implements IChannelControlBlock {
     private final ChannelSet cSet;
 
     private final int channelId;
 
-    private final ReadInterface ri;
+    private final IChannelReadInterface ri;
 
-    private final WriteInterface wi;
+    private final IChannelWriteInterface wi;
 
     private final AtomicBoolean localClose;
 
@@ -56,298 +51,38 @@
 
     private final AtomicBoolean remoteCloseAck;
 
-    ChannelControlBlock(ChannelSet cSet, int channelId) {
+    ChannelControlBlock(ChannelSet cSet, int channelId, IChannelInterfaceFactory interfaceFactory) {
         this.cSet = cSet;
         this.channelId = channelId;
-        this.ri = new ReadInterface();
-        this.wi = new WriteInterface();
         localClose = new AtomicBoolean();
         localCloseAck = new AtomicBoolean();
         remoteClose = new AtomicBoolean();
         remoteCloseAck = new AtomicBoolean();
+        this.ri = interfaceFactory.createReadInterface(this);
+        this.wi = interfaceFactory.createWriteInterface(this);
     }
 
-    int getChannelId() {
+    @Override
+    public int getChannelId() {
         return channelId;
     }
 
-    /**
-     * Get the read inderface of this channel.
-     *
-     * @return the read interface.
-     */
+    @Override
     public IChannelReadInterface getReadInterface() {
         return ri;
     }
 
-    /**
-     * Get the write interface of this channel.
-     *
-     * @return the write interface.
-     */
+    @Override
     public IChannelWriteInterface getWriteInterface() {
         return wi;
     }
 
-    private final class ReadInterface implements IChannelReadInterface {
-        private final Deque<ByteBuffer> riEmptyStack;
-
-        private final IBufferAcceptor eba = new IBufferAcceptor() {
-            @Override
-            public void accept(ByteBuffer buffer) {
-                int delta = buffer.remaining();
-                synchronized (ChannelControlBlock.this) {
-                    if (remoteClose.get()) {
-                        return;
-                    }
-                    riEmptyStack.push(buffer);
-                }
-                cSet.addPendingCredits(channelId, delta);
-            }
-        };
-
-        private ICloseableBufferAcceptor fba;
-
-        private volatile int credits;
-
-        private ByteBuffer currentReadBuffer;
-
-        private IBufferFactory bufferFactory;
-
-        ReadInterface() {
-            riEmptyStack = new ArrayDeque<ByteBuffer>();
-            credits = 0;
-        }
-
-        @Override
-        public void setBufferFactory(IBufferFactory bufferFactory, int limit, int frameSize) {
-            this.bufferFactory = bufferFactory;
-            cSet.addPendingCredits(channelId, limit * frameSize);
-        }
-
-        @Override
-        public void setFullBufferAcceptor(ICloseableBufferAcceptor fullBufferAcceptor) {
-            fba = fullBufferAcceptor;
-        }
-
-        @Override
-        public IBufferAcceptor getEmptyBufferAcceptor() {
-            return eba;
-        }
-
-        int read(SocketChannel sc, int size) throws IOException, NetException {
-            while (true) {
-                if (size <= 0) {
-                    return size;
-                }
-                if (currentReadBuffer == null) {
-                    currentReadBuffer = riEmptyStack.poll();
-                    //if current buffer == null and limit not reached
-                    // factory.createBuffer factory
-                    if (currentReadBuffer == null) {
-                        currentReadBuffer = bufferFactory.createBuffer();
-                    }
-                    assert currentReadBuffer != null;
-                }
-                int rSize = Math.min(size, currentReadBuffer.remaining());
-                if (rSize > 0) {
-                    currentReadBuffer.limit(currentReadBuffer.position() + rSize);
-                    int len;
-                    try {
-                        len = sc.read(currentReadBuffer);
-                        if (len < 0) {
-                            throw new NetException("Socket Closed");
-                        }
-                    } finally {
-                        currentReadBuffer.limit(currentReadBuffer.capacity());
-                    }
-                    size -= len;
-                    if (len < rSize) {
-                        return size;
-                    }
-                } else {
-                    return size;
-                }
-                if (currentReadBuffer.remaining() <= 0) {
-                    flush();
-                }
-            }
-        }
-
-        void flush() {
-            if (currentReadBuffer != null) {
-                currentReadBuffer.flip();
-                fba.accept(currentReadBuffer);
-                currentReadBuffer = null;
-            }
-        }
-    }
-
-    private final class WriteInterface implements IChannelWriteInterface {
-        private final Queue<ByteBuffer> wiFullQueue;
-
-        private boolean channelWritabilityState;
-
-        private IBufferFactory bufferFactory;
-
-        private final ICloseableBufferAcceptor fba = new ICloseableBufferAcceptor() {
-            @Override
-            public void accept(ByteBuffer buffer) {
-                synchronized (ChannelControlBlock.this) {
-                    wiFullQueue.add(buffer);
-                    adjustChannelWritability();
-                }
-            }
-
-            @Override
-            public void close() {
-                synchronized (ChannelControlBlock.this) {
-                    if (eos) {
-                        if (LOGGER.isLoggable(Level.WARNING)) {
-                            LOGGER.warning("Received duplicate close() on channel: " + channelId);
-                        }
-                        return;
-                    }
-                    eos = true;
-                    adjustChannelWritability();
-                }
-            }
-
-            @Override
-            public void error(int ecode) {
-                synchronized (ChannelControlBlock.this) {
-                    WriteInterface.this.ecode = ecode;
-                    adjustChannelWritability();
-                }
-            }
-        };
-
-        private IBufferAcceptor eba;
-
-        private int credits;
-
-        private boolean eos;
-
-        private boolean eosSent;
-
-        private int ecode;
-
-        private boolean ecodeSent;
-
-        private ByteBuffer currentWriteBuffer;
-
-        WriteInterface() {
-            wiFullQueue = new ArrayDeque<ByteBuffer>();
-            credits = 0;
-            eos = false;
-            eosSent = false;
-            ecode = -1;
-            ecodeSent = false;
-        }
-
-        @Override
-        public void setBufferFactory(IBufferFactory bufferFactory, int limit, int frameSize) {
-            this.bufferFactory = bufferFactory;
-            if (!channelWritabilityState) {
-                cSet.markPendingWrite(channelId);
-            }
-            channelWritabilityState = true;
-            if (eos) {
-                if (LOGGER.isLoggable(Level.WARNING)) {
-                    LOGGER.warning("Received duplicate close() on channel: " + channelId);
-                }
-                return;
-            }
-            eos = true;
-        }
-
-        @Override
-        public void setEmptyBufferAcceptor(IBufferAcceptor emptyBufferAcceptor) {
-            eba = emptyBufferAcceptor;
-        }
-
-        @Override
-        public ICloseableBufferAcceptor getFullBufferAcceptor() {
-            return fba;
-        }
-
-        void write(MultiplexedConnection.WriterState writerState) throws NetException {
-            if (currentWriteBuffer == null) {
-                currentWriteBuffer = wiFullQueue.poll();
-            }
-            if (currentWriteBuffer != null) {
-                int size = Math.min(currentWriteBuffer.remaining(), credits);
-                if (size > 0) {
-                    credits -= size;
-                    writerState.command.setChannelId(channelId);
-                    writerState.command.setCommandType(MuxDemuxCommand.CommandType.DATA);
-                    writerState.command.setData(size);
-                    writerState.reset(currentWriteBuffer, size, ChannelControlBlock.this);
-                } else {
-                    adjustChannelWritability();
-                }
-            } else if (ecode >= 0 && !ecodeSent) {
-                writerState.command.setChannelId(channelId);
-                writerState.command.setCommandType(MuxDemuxCommand.CommandType.ERROR);
-                writerState.command.setData(ecode);
-                writerState.reset(null, 0, null);
-                ecodeSent = true;
-                localClose.set(true);
-                adjustChannelWritability();
-            } else if (eos && !eosSent) {
-                writerState.command.setChannelId(channelId);
-                writerState.command.setCommandType(MuxDemuxCommand.CommandType.CLOSE_CHANNEL);
-                writerState.command.setData(0);
-                writerState.reset(null, 0, null);
-                eosSent = true;
-                localClose.set(true);
-                adjustChannelWritability();
-            }
-        }
-
-        void writeComplete() {
-            if (currentWriteBuffer.remaining() <= 0) {
-                currentWriteBuffer.clear();
-                eba.accept(currentWriteBuffer);
-                currentWriteBuffer = null;
-                adjustChannelWritability();
-            }
-        }
-
-        private boolean computeWritability() {
-            boolean writableDataPresent = currentWriteBuffer != null || !wiFullQueue.isEmpty();
-            if (writableDataPresent) {
-                return credits > 0;
-            }
-            if (eos && !eosSent) {
-                return true;
-            }
-            if (ecode >= 0 && !ecodeSent) {
-                return true;
-            }
-            return false;
-        }
-
-        void adjustChannelWritability() {
-            boolean writable = computeWritability();
-            if (writable) {
-                if (!channelWritabilityState) {
-                    cSet.markPendingWrite(channelId);
-                }
-            } else {
-                if (channelWritabilityState) {
-                    cSet.unmarkPendingWrite(channelId);
-                }
-            }
-            channelWritabilityState = writable;
-        }
-    }
-
-    synchronized void write(MultiplexedConnection.WriterState writerState) throws NetException {
+    synchronized void write(WriterState writerState) throws NetException {
         wi.write(writerState);
     }
 
-    synchronized void writeComplete() {
+    @Override
+    public synchronized void writeComplete() {
         wi.writeComplete();
     }
 
@@ -356,21 +91,22 @@
     }
 
     int getReadCredits() {
-        return ri.credits;
+        return ri.getCredits();
     }
 
     void setReadCredits(int credits) {
-        this.ri.credits = credits;
+        ri.setReadCredits(credits);
     }
 
-    synchronized void addWriteCredits(int delta) {
-        wi.credits += delta;
+    @Override
+    public synchronized void addWriteCredits(int delta) {
+        wi.addCredits(delta);
         wi.adjustChannelWritability();
     }
 
     synchronized void reportRemoteEOS() {
         ri.flush();
-        ri.fba.close();
+        ri.getFullBufferAcceptor().close();
         remoteClose.set(true);
     }
 
@@ -388,7 +124,7 @@
 
     synchronized void reportRemoteError(int ecode) {
         ri.flush();
-        ri.fba.error(ecode);
+        ri.getFullBufferAcceptor().error(ecode);
         remoteClose.set(true);
     }
 
@@ -397,9 +133,34 @@
     }
 
     @Override
+    public boolean isRemotelyClosed() {
+        return remoteCloseAck.get();
+    }
+
+    @Override
+    public void reportLocalEOS() {
+        localClose.set(true);
+    }
+
+    @Override
+    public void addPendingCredits(int credit) {
+        cSet.addPendingCredits(channelId, credit);
+    }
+
+    @Override
+    public void unmarkPendingWrite() {
+        cSet.unmarkPendingWrite(channelId);
+    }
+
+    @Override
+    public void markPendingWrite() {
+        cSet.markPendingWrite(channelId);
+    }
+
+    @Override
     public String toString() {
         return "Channel:" + channelId + "[localClose: " + localClose + " localCloseAck: " + localCloseAck
-                + " remoteClose: " + remoteClose + " remoteCloseAck:" + remoteCloseAck + " readCredits: " + ri.credits
-                + " writeCredits: " + wi.credits + "]";
+                + " remoteClose: " + remoteClose + " remoteCloseAck:" + remoteCloseAck + " readCredits: "
+                + ri.getCredits() + " writeCredits: " + wi.getCredits() + "]";
     }
 }
diff --git a/hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/protocols/muxdemux/ChannelSet.java b/hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/protocols/muxdemux/ChannelSet.java
index 52f3417..49bb292 100644
--- a/hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/protocols/muxdemux/ChannelSet.java
+++ b/hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/protocols/muxdemux/ChannelSet.java
@@ -23,7 +23,9 @@
 import java.util.logging.Level;
 import java.util.logging.Logger;
 
-import org.apache.hyracks.net.exceptions.NetException;
+import org.apache.hyracks.api.comm.IChannelInterfaceFactory;
+import org.apache.hyracks.api.comm.MuxDemuxCommand;
+import org.apache.hyracks.api.exceptions.NetException;
 
 public class ChannelSet {
     private static final Logger LOGGER = Logger.getLogger(ChannelSet.class.getName());
@@ -224,7 +226,8 @@
         }
         assert idx < ccbArray.length;
         assert !allocationBitmap.get(idx);
-        ChannelControlBlock channel = new ChannelControlBlock(this, idx);
+        IChannelInterfaceFactory channelInterfaceFactory = mConn.getChannelInterfaceFactory();
+        ChannelControlBlock channel = new ChannelControlBlock(this, idx, channelInterfaceFactory);
         ccbArray[idx] = channel;
         allocationBitmap.set(idx);
         ++openChannelCount;
diff --git a/hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/protocols/muxdemux/FullFrameChannelInterfaceFactory.java b/hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/protocols/muxdemux/FullFrameChannelInterfaceFactory.java
new file mode 100644
index 0000000..459fe51
--- /dev/null
+++ b/hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/protocols/muxdemux/FullFrameChannelInterfaceFactory.java
@@ -0,0 +1,39 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements.  See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership.  The ASF licenses this file
+ * to you 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.apache.hyracks.net.protocols.muxdemux;
+
+import org.apache.hyracks.api.comm.IChannelControlBlock;
+import org.apache.hyracks.api.comm.IChannelInterfaceFactory;
+import org.apache.hyracks.api.comm.IChannelReadInterface;
+import org.apache.hyracks.api.comm.IChannelWriteInterface;
+
+public class FullFrameChannelInterfaceFactory implements IChannelInterfaceFactory {
+
+    public static final IChannelInterfaceFactory INSTANCE = new FullFrameChannelInterfaceFactory();
+
+    @Override
+    public IChannelReadInterface createReadInterface(IChannelControlBlock cbb) {
+        return new FullFrameChannelReadInterface(cbb);
+    }
+
+    @Override
+    public IChannelWriteInterface createWriteInterface(IChannelControlBlock cbb) {
+        return new FullFrameChannelWriteInterface(cbb);
+    }
+}
diff --git a/hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/protocols/muxdemux/FullFrameChannelReadInterface.java b/hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/protocols/muxdemux/FullFrameChannelReadInterface.java
new file mode 100644
index 0000000..432382a
--- /dev/null
+++ b/hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/protocols/muxdemux/FullFrameChannelReadInterface.java
@@ -0,0 +1,97 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements.  See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership.  The ASF licenses this file
+ * to you 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.apache.hyracks.net.protocols.muxdemux;
+
+import java.io.IOException;
+import java.nio.ByteBuffer;
+import java.nio.channels.SocketChannel;
+import java.util.ArrayDeque;
+import java.util.Deque;
+
+import org.apache.hyracks.api.comm.IBufferFactory;
+import org.apache.hyracks.api.comm.IChannelControlBlock;
+import org.apache.hyracks.api.exceptions.NetException;
+
+public class FullFrameChannelReadInterface extends AbstractChannelReadInterface {
+
+    private final Deque<ByteBuffer> riEmptyStack;
+    private final IChannelControlBlock ccb;
+
+    FullFrameChannelReadInterface(IChannelControlBlock ccb) {
+        this.ccb = ccb;
+        riEmptyStack = new ArrayDeque<>();
+        credits = 0;
+
+        emptyBufferAcceptor = buffer -> {
+            int delta = buffer.remaining();
+            synchronized (ccb) {
+                if (ccb.isRemotelyClosed()) {
+                    return;
+                }
+                riEmptyStack.push(buffer);
+            }
+            ccb.addPendingCredits(delta);
+        };
+    }
+
+    @Override
+    public int read(SocketChannel sc, int size) throws IOException, NetException {
+        while (true) {
+            if (size <= 0) {
+                return size;
+            }
+            if (currentReadBuffer == null) {
+                currentReadBuffer = riEmptyStack.poll();
+                //if current buffer == null and limit not reached
+                // factory.createBuffer factory
+                if (currentReadBuffer == null) {
+                    currentReadBuffer = bufferFactory.createBuffer();
+                }
+            }
+            int rSize = Math.min(size, currentReadBuffer.remaining());
+            if (rSize > 0) {
+                currentReadBuffer.limit(currentReadBuffer.position() + rSize);
+                int len;
+                try {
+                    len = sc.read(currentReadBuffer);
+                    if (len < 0) {
+                        throw new NetException("Socket Closed");
+                    }
+                } finally {
+                    currentReadBuffer.limit(currentReadBuffer.capacity());
+                }
+                size -= len;
+                if (len < rSize) {
+                    return size;
+                }
+            } else {
+                return size;
+            }
+            if (currentReadBuffer.remaining() <= 0) {
+                flush();
+            }
+        }
+    }
+
+    @Override
+    public void setBufferFactory(IBufferFactory bufferFactory, int limit, int frameSize) {
+        super.setBufferFactory(bufferFactory, limit, frameSize);
+        ccb.addPendingCredits(limit * frameSize);
+    }
+}
\ No newline at end of file
diff --git a/hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/protocols/muxdemux/FullFrameChannelWriteInterface.java b/hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/protocols/muxdemux/FullFrameChannelWriteInterface.java
new file mode 100644
index 0000000..418ebd7
--- /dev/null
+++ b/hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/protocols/muxdemux/FullFrameChannelWriteInterface.java
@@ -0,0 +1,87 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements.  See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership.  The ASF licenses this file
+ * to you 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.apache.hyracks.net.protocols.muxdemux;
+
+import java.util.logging.Level;
+import java.util.logging.Logger;
+
+import org.apache.hyracks.api.comm.IBufferFactory;
+import org.apache.hyracks.api.comm.IChannelControlBlock;
+import org.apache.hyracks.api.comm.IConnectionWriterState;
+import org.apache.hyracks.api.comm.MuxDemuxCommand;
+import org.apache.hyracks.api.exceptions.NetException;
+
+public class FullFrameChannelWriteInterface extends AbstractChannelWriteInterface {
+
+    private static final Logger LOGGER = Logger.getLogger(FullFrameChannelWriteInterface.class.getName());
+
+    FullFrameChannelWriteInterface(IChannelControlBlock ccb) {
+        super(ccb);
+    }
+
+    @Override
+    public void write(IConnectionWriterState writerState) throws NetException {
+        if (currentWriteBuffer == null) {
+            currentWriteBuffer = wiFullQueue.poll();
+        }
+        if (currentWriteBuffer != null) {
+            int size = Math.min(currentWriteBuffer.remaining(), credits);
+            if (size > 0) {
+                credits -= size;
+                writerState.getCommand().setChannelId(channelId);
+                writerState.getCommand().setCommandType(MuxDemuxCommand.CommandType.DATA);
+                writerState.getCommand().setData(size);
+                writerState.reset(currentWriteBuffer, size, ccb);
+            } else {
+                adjustChannelWritability();
+            }
+        } else if (ecode >= 0 && !ecodeSent) {
+            writerState.getCommand().setChannelId(channelId);
+            writerState.getCommand().setCommandType(MuxDemuxCommand.CommandType.ERROR);
+            writerState.getCommand().setData(ecode);
+            writerState.reset(null, 0, null);
+            ecodeSent = true;
+            ccb.reportLocalEOS();
+            adjustChannelWritability();
+        } else if (eos && !eosSent) {
+            writerState.getCommand().setChannelId(channelId);
+            writerState.getCommand().setCommandType(MuxDemuxCommand.CommandType.CLOSE_CHANNEL);
+            writerState.getCommand().setData(0);
+            writerState.reset(null, 0, null);
+            eosSent = true;
+            ccb.reportLocalEOS();
+            adjustChannelWritability();
+        }
+    }
+
+    @Override
+    public void setBufferFactory(IBufferFactory bufferFactory, int limit, int frameSize) {
+        if (!channelWritabilityState) {
+            ccb.markPendingWrite();
+        }
+        channelWritabilityState = true;
+        if (eos) {
+            if (LOGGER.isLoggable(Level.WARNING)) {
+                LOGGER.warning("Received duplicate close() on channel: " + channelId);
+            }
+            return;
+        }
+        eos = true;
+    }
+}
\ No newline at end of file
diff --git a/hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/protocols/muxdemux/MultiplexedConnection.java b/hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/protocols/muxdemux/MultiplexedConnection.java
index 866f2c6..ad8aabd 100644
--- a/hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/protocols/muxdemux/MultiplexedConnection.java
+++ b/hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/protocols/muxdemux/MultiplexedConnection.java
@@ -26,7 +26,11 @@
 import java.util.logging.Level;
 import java.util.logging.Logger;
 
-import org.apache.hyracks.net.exceptions.NetException;
+import org.apache.hyracks.api.comm.IChannelControlBlock;
+import org.apache.hyracks.api.comm.IChannelInterfaceFactory;
+import org.apache.hyracks.api.comm.IConnectionWriterState;
+import org.apache.hyracks.api.comm.MuxDemuxCommand;
+import org.apache.hyracks.api.exceptions.NetException;
 import org.apache.hyracks.net.protocols.tcp.ITCPConnectionEventListener;
 import org.apache.hyracks.net.protocols.tcp.TCPConnection;
 
@@ -121,8 +125,8 @@
     }
 
     @Override
-    public void notifyIOReady(TCPConnection connection, boolean readable, boolean writable) throws IOException,
-            NetException {
+    public void notifyIOReady(TCPConnection connection, boolean readable, boolean writable)
+            throws IOException, NetException {
         if (readable) {
             driveReaderStateMachine();
         }
@@ -157,7 +161,7 @@
         return channel;
     }
 
-    class WriterState {
+    class WriterState implements IConnectionWriterState {
         private final ByteBuffer cmdWriteBuffer;
 
         final MuxDemuxCommand command;
@@ -166,7 +170,7 @@
 
         private int pendingWriteSize;
 
-        private ChannelControlBlock ccb;
+        private IChannelControlBlock ccb;
 
         public WriterState() {
             cmdWriteBuffer = ByteBuffer.allocateDirect(MuxDemuxCommand.COMMAND_SIZE);
@@ -179,7 +183,8 @@
             return cmdWriteBuffer.remaining() > 0 || (pendingBuffer != null && pendingWriteSize > 0);
         }
 
-        void reset(ByteBuffer pendingBuffer, int pendingWriteSize, ChannelControlBlock ccb) {
+        @Override
+        public void reset(ByteBuffer pendingBuffer, int pendingWriteSize, IChannelControlBlock ccb) {
             cmdWriteBuffer.clear();
             command.write(cmdWriteBuffer);
             cmdWriteBuffer.flip();
@@ -221,6 +226,11 @@
                 ccb = null;
             }
             return true;
+        }
+
+        @Override
+        public MuxDemuxCommand getCommand() {
+            return command;
         }
     }
 
@@ -414,4 +424,8 @@
             readerState.reset();
         }
     }
-}
+
+    public IChannelInterfaceFactory getChannelInterfaceFactory() {
+        return muxDemux.getChannelInterfaceFactory();
+    }
+}
\ No newline at end of file
diff --git a/hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/protocols/muxdemux/MuxDemux.java b/hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/protocols/muxdemux/MuxDemux.java
index 0ea7b13..7e6c8d7 100644
--- a/hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/protocols/muxdemux/MuxDemux.java
+++ b/hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/protocols/muxdemux/MuxDemux.java
@@ -23,7 +23,8 @@
 import java.util.HashMap;
 import java.util.Map;
 
-import org.apache.hyracks.net.exceptions.NetException;
+import org.apache.hyracks.api.comm.IChannelInterfaceFactory;
+import org.apache.hyracks.api.exceptions.NetException;
 import org.apache.hyracks.net.protocols.tcp.ITCPConnectionListener;
 import org.apache.hyracks.net.protocols.tcp.TCPConnection;
 import org.apache.hyracks.net.protocols.tcp.TCPEndpoint;
@@ -48,6 +49,8 @@
 
     private final MuxDemuxPerformanceCounters perfCounters;
 
+    private final IChannelInterfaceFactory channelInterfaceFatory;
+
     /**
      * Constructor.
      *
@@ -61,11 +64,12 @@
      *            - Maximum number of connection attempts
      */
     public MuxDemux(InetSocketAddress localAddress, IChannelOpenListener listener, int nThreads,
-            int maxConnectionAttempts) {
+            int maxConnectionAttempts, IChannelInterfaceFactory channelInterfaceFatory) {
         this.localAddress = localAddress;
         this.channelOpenListener = listener;
         this.maxConnectionAttempts = maxConnectionAttempts;
-        connectionMap = new HashMap<InetSocketAddress, MultiplexedConnection>();
+        this.channelInterfaceFatory = channelInterfaceFatory;
+        connectionMap = new HashMap<>();
         this.tcpEndpoint = new TCPEndpoint(new ITCPConnectionListener() {
             @Override
             public void connectionEstablished(TCPConnection connection) {
@@ -129,7 +133,7 @@
      * @throws NetException
      */
     public MultiplexedConnection connect(InetSocketAddress remoteAddress) throws InterruptedException, NetException {
-        MultiplexedConnection mConn = null;
+        MultiplexedConnection mConn;
         synchronized (this) {
             mConn = connectionMap.get(remoteAddress);
             if (mConn == null) {
@@ -163,4 +167,14 @@
     public MuxDemuxPerformanceCounters getPerformanceCounters() {
         return perfCounters;
     }
-}
+
+    /**
+     * Gets the channel interface factory associated with channels
+     * created by this {@link MuxDemux}.
+     *
+     * @return
+     */
+    public IChannelInterfaceFactory getChannelInterfaceFactory() {
+        return channelInterfaceFatory;
+    }
+}
\ No newline at end of file
diff --git a/hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/protocols/tcp/ITCPConnectionEventListener.java b/hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/protocols/tcp/ITCPConnectionEventListener.java
index 59b1fbb..eaff865 100644
--- a/hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/protocols/tcp/ITCPConnectionEventListener.java
+++ b/hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/protocols/tcp/ITCPConnectionEventListener.java
@@ -20,11 +20,11 @@
 
 import java.io.IOException;
 
-import org.apache.hyracks.net.exceptions.NetException;
+import org.apache.hyracks.api.exceptions.NetException;
 
 public interface ITCPConnectionEventListener {
-    public void notifyIOReady(TCPConnection connection, boolean readable, boolean writable) throws IOException,
-            NetException;
+    public void notifyIOReady(TCPConnection connection, boolean readable, boolean writable)
+            throws IOException, NetException;
 
     public void notifyIOError(Exception e);
 }
diff --git a/hyracks-fullstack/hyracks/hyracks-net/src/test/java/org/apache/hyracks/net/tests/NetTest.java b/hyracks-fullstack/hyracks/hyracks-net/src/test/java/org/apache/hyracks/net/tests/NetTest.java
index f1f3c4a..ab018db 100644
--- a/hyracks-fullstack/hyracks/hyracks-net/src/test/java/org/apache/hyracks/net/tests/NetTest.java
+++ b/hyracks-fullstack/hyracks/hyracks-net/src/test/java/org/apache/hyracks/net/tests/NetTest.java
@@ -25,16 +25,16 @@
 import java.util.concurrent.Semaphore;
 import java.util.concurrent.atomic.AtomicBoolean;
 
-import junit.framework.Assert;
-
-import org.junit.Test;
-
-import org.apache.hyracks.net.buffers.IBufferAcceptor;
-import org.apache.hyracks.net.buffers.ICloseableBufferAcceptor;
+import org.apache.hyracks.api.comm.IBufferAcceptor;
+import org.apache.hyracks.api.comm.ICloseableBufferAcceptor;
 import org.apache.hyracks.net.protocols.muxdemux.ChannelControlBlock;
+import org.apache.hyracks.net.protocols.muxdemux.FullFrameChannelInterfaceFactory;
 import org.apache.hyracks.net.protocols.muxdemux.IChannelOpenListener;
 import org.apache.hyracks.net.protocols.muxdemux.MultiplexedConnection;
 import org.apache.hyracks.net.protocols.muxdemux.MuxDemux;
+import org.junit.Test;
+
+import junit.framework.Assert;
 
 public class NetTest {
     @Test
@@ -162,7 +162,8 @@
                 }.start();
             }
         };
-        return new MuxDemux(new InetSocketAddress("127.0.0.1", 0), md1OpenListener, 1, 5);
+        return new MuxDemux(new InetSocketAddress("127.0.0.1", 0), md1OpenListener, 1, 5,
+                FullFrameChannelInterfaceFactory.INSTANCE);
     }
 
     private class ChannelIO {
diff --git a/hyracks-fullstack/hyracks/hyracks-test-support/src/main/java/org/apache/hyracks/test/support/TestNCApplicationContext.java b/hyracks-fullstack/hyracks/hyracks-test-support/src/main/java/org/apache/hyracks/test/support/TestNCApplicationContext.java
index c6a4430..e4470ee 100644
--- a/hyracks-fullstack/hyracks/hyracks-test-support/src/main/java/org/apache/hyracks/test/support/TestNCApplicationContext.java
+++ b/hyracks-fullstack/hyracks/hyracks-test-support/src/main/java/org/apache/hyracks/test/support/TestNCApplicationContext.java
@@ -24,6 +24,7 @@
 import org.apache.hyracks.api.application.IApplicationConfig;
 import org.apache.hyracks.api.application.INCApplicationContext;
 import org.apache.hyracks.api.application.IStateDumpHandler;
+import org.apache.hyracks.api.comm.IChannelInterfaceFactory;
 import org.apache.hyracks.api.context.IHyracksRootContext;
 import org.apache.hyracks.api.job.IJobSerializerDeserializerContainer;
 import org.apache.hyracks.api.lifecycle.ILifeCycleComponentManager;
@@ -140,4 +141,14 @@
     public IControllerService getControllerService() {
         return null;
     }
+
+    @Override
+    public IChannelInterfaceFactory getMessagingChannelInterfaceFactory() {
+        return null;
+    }
+
+    @Override
+    public void setMessagingChannelInterfaceFactory(IChannelInterfaceFactory interfaceFactory) {
+        // do nothing
+    }
 }

-- 
To view, visit https://asterix-gerrit.ics.uci.edu/897
To unsubscribe, visit https://asterix-gerrit.ics.uci.edu/settings

Gerrit-MessageType: merged
Gerrit-Change-Id: I5c0bd7c11c1e78954ebceff49cb274d8073a64bd
Gerrit-PatchSet: 10
Gerrit-Project: asterixdb
Gerrit-Branch: master
Gerrit-Owner: Murtadha Hubail <hu...@gmail.com>
Gerrit-Reviewer: Jenkins <je...@fulliautomatix.ics.uci.edu>
Gerrit-Reviewer: Murtadha Hubail <hu...@gmail.com>
Gerrit-Reviewer: Till Westmann <ti...@apache.org>
Gerrit-Reviewer: Yingyi Bu <bu...@gmail.com>
Gerrit-Reviewer: abdullah alamoudi <ba...@gmail.com>

Change in asterixdb[master]: Introduce MessagingNetworkManager for NC2NC AppMessaging

Posted by "Jenkins (Code Review)" <do...@asterixdb.incubator.apache.org>.
Jenkins has posted comments on this change.

Change subject: Introduce MessagingNetworkManager for NC2NC AppMessaging
......................................................................


Patch Set 6:

WARNING: THIS CHANGE CONTAINS CROSS-PRODUCT CHANGES IN:
* asterixdb
* hyracks-fullstack

PLEASE REVIEW CAREFULLY AND LOOK FOR API CHANGES!

-- 
To view, visit https://asterix-gerrit.ics.uci.edu/897
To unsubscribe, visit https://asterix-gerrit.ics.uci.edu/settings

Gerrit-MessageType: comment
Gerrit-Change-Id: I5c0bd7c11c1e78954ebceff49cb274d8073a64bd
Gerrit-PatchSet: 6
Gerrit-Project: asterixdb
Gerrit-Branch: master
Gerrit-Owner: Murtadha Hubail <hu...@gmail.com>
Gerrit-Reviewer: Jenkins <je...@fulliautomatix.ics.uci.edu>
Gerrit-Reviewer: Till Westmann <ti...@apache.org>
Gerrit-Reviewer: Yingyi Bu <bu...@gmail.com>
Gerrit-Reviewer: abdullah alamoudi <ba...@gmail.com>
Gerrit-HasComments: No

Change in asterixdb[master]: Introduce MessagingNetworkManager for NC2NC AppMessaging

Posted by "Jenkins (Code Review)" <do...@asterixdb.incubator.apache.org>.
Jenkins has posted comments on this change.

Change subject: Introduce MessagingNetworkManager for NC2NC AppMessaging
......................................................................


Patch Set 5:

WARNING: THIS CHANGE CONTAINS CROSS-PRODUCT CHANGES IN:
* asterixdb
* hyracks-fullstack

PLEASE REVIEW CAREFULLY AND LOOK FOR API CHANGES!

-- 
To view, visit https://asterix-gerrit.ics.uci.edu/897
To unsubscribe, visit https://asterix-gerrit.ics.uci.edu/settings

Gerrit-MessageType: comment
Gerrit-Change-Id: I5c0bd7c11c1e78954ebceff49cb274d8073a64bd
Gerrit-PatchSet: 5
Gerrit-Project: asterixdb
Gerrit-Branch: master
Gerrit-Owner: Murtadha Hubail <hu...@gmail.com>
Gerrit-Reviewer: Jenkins <je...@fulliautomatix.ics.uci.edu>
Gerrit-Reviewer: Till Westmann <ti...@apache.org>
Gerrit-Reviewer: Yingyi Bu <bu...@gmail.com>
Gerrit-Reviewer: abdullah alamoudi <ba...@gmail.com>
Gerrit-HasComments: No

Change in asterixdb[master]: Introduce MessagingNetworkManager for NC2NC AppMessaging

Posted by "Jenkins (Code Review)" <do...@asterixdb.incubator.apache.org>.
Jenkins has posted comments on this change.

Change subject: Introduce MessagingNetworkManager for NC2NC AppMessaging
......................................................................


Patch Set 9:

Integration Tests Started https://asterix-jenkins.ics.uci.edu/job/asterix-gerrit-integration-tests/470/

-- 
To view, visit https://asterix-gerrit.ics.uci.edu/897
To unsubscribe, visit https://asterix-gerrit.ics.uci.edu/settings

Gerrit-MessageType: comment
Gerrit-Change-Id: I5c0bd7c11c1e78954ebceff49cb274d8073a64bd
Gerrit-PatchSet: 9
Gerrit-Project: asterixdb
Gerrit-Branch: master
Gerrit-Owner: Murtadha Hubail <hu...@gmail.com>
Gerrit-Reviewer: Jenkins <je...@fulliautomatix.ics.uci.edu>
Gerrit-Reviewer: Murtadha Hubail <hu...@gmail.com>
Gerrit-Reviewer: Till Westmann <ti...@apache.org>
Gerrit-Reviewer: Yingyi Bu <bu...@gmail.com>
Gerrit-Reviewer: abdullah alamoudi <ba...@gmail.com>
Gerrit-HasComments: No

Change in asterixdb[master]: Introduce MessagingNetworkManager for NC2NC AppMessaging

Posted by "Till Westmann (Code Review)" <do...@asterixdb.incubator.apache.org>.
Till Westmann has posted comments on this change.

Change subject: Introduce MessagingNetworkManager for NC2NC AppMessaging
......................................................................


Patch Set 6:

(7 comments)

https://asterix-gerrit.ics.uci.edu/#/c/897/6/asterixdb/asterix-app/src/main/java/org/apache/asterix/messaging/MessagingChannelInterfaceFactory.java
File asterixdb/asterix-app/src/main/java/org/apache/asterix/messaging/MessagingChannelInterfaceFactory.java:

Line 90:                     if (LOGGER.isLoggable(Level.WARNING)) {
Maybe this should just throw an exception? It seems that everywhere we call this method we wither assert that the buffer is not null or we access the buffer without checking (getting an NPE it it ever were null). A NetException might work ..


https://asterix-gerrit.ics.uci.edu/#/c/897/6/asterixdb/asterix-app/src/main/java/org/apache/asterix/messaging/MessagingChannelReadInterface.java
File asterixdb/asterix-app/src/main/java/org/apache/asterix/messaging/MessagingChannelReadInterface.java:

Line 31: public class MessagingChannelReadInterface implements IChannelReadInterface {
It seems that there's a lot of overlap with FullFrameChannelReadInterface. Is there an opportunity for some reuse (in a reasonably generic way - i.e. no hack to exactly support this implementation).


https://asterix-gerrit.ics.uci.edu/#/c/897/6/asterixdb/asterix-app/src/main/java/org/apache/asterix/messaging/MessagingChannelWriteInterface.java
File asterixdb/asterix-app/src/main/java/org/apache/asterix/messaging/MessagingChannelWriteInterface.java:

Line 36: public class MessagingChannelWriteInterface implements IChannelWriteInterface {
It seems that there's a lot of overlap with FullFrameChannelWriteInterface. Is there an opportunity for some reuse (in a reasonably generic way - i.e. no hack to exactly support this implementation).


https://asterix-gerrit.ics.uci.edu/#/c/897/6/asterixdb/asterix-app/src/main/java/org/apache/asterix/messaging/NCMessageBroker.java
File asterixdb/asterix-app/src/main/java/org/apache/asterix/messaging/NCMessageBroker.java:

Line 66:     private final MessageDeliveryService msgDeliverySvc;
Do we need to keep this as a member?


Line 288:                     Thread.currentThread().interrupt();
Is there a reason why we shouldn't exit run(), if the thread gets interrupted?


Line 291:                         LOGGER.log(Level.WARNING, "Could not process message", e);
Could we add some info (e.g. the message id) to the warning?


https://asterix-gerrit.ics.uci.edu/#/c/897/6/hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/IChannelReadInterface.java
File hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/IChannelReadInterface.java:

Line 98:     public IBufferFactory getBufferFactor();
a/getBufferFactor/getBufferFactory/


-- 
To view, visit https://asterix-gerrit.ics.uci.edu/897
To unsubscribe, visit https://asterix-gerrit.ics.uci.edu/settings

Gerrit-MessageType: comment
Gerrit-Change-Id: I5c0bd7c11c1e78954ebceff49cb274d8073a64bd
Gerrit-PatchSet: 6
Gerrit-Project: asterixdb
Gerrit-Branch: master
Gerrit-Owner: Murtadha Hubail <hu...@gmail.com>
Gerrit-Reviewer: Jenkins <je...@fulliautomatix.ics.uci.edu>
Gerrit-Reviewer: Murtadha Hubail <hu...@gmail.com>
Gerrit-Reviewer: Till Westmann <ti...@apache.org>
Gerrit-Reviewer: Yingyi Bu <bu...@gmail.com>
Gerrit-Reviewer: abdullah alamoudi <ba...@gmail.com>
Gerrit-HasComments: Yes

Change in asterixdb[master]: Introduce MessagingNetworkManager for NC2NC AppMessaging

Posted by "Jenkins (Code Review)" <do...@asterixdb.incubator.apache.org>.
Jenkins has posted comments on this change.

Change subject: Introduce MessagingNetworkManager for NC2NC AppMessaging
......................................................................


Patch Set 8: Integration-Tests-1

Integration Tests Failed

https://asterix-jenkins.ics.uci.edu/job/asterix-gerrit-integration-tests/465/ : UNSTABLE

-- 
To view, visit https://asterix-gerrit.ics.uci.edu/897
To unsubscribe, visit https://asterix-gerrit.ics.uci.edu/settings

Gerrit-MessageType: comment
Gerrit-Change-Id: I5c0bd7c11c1e78954ebceff49cb274d8073a64bd
Gerrit-PatchSet: 8
Gerrit-Project: asterixdb
Gerrit-Branch: master
Gerrit-Owner: Murtadha Hubail <hu...@gmail.com>
Gerrit-Reviewer: Jenkins <je...@fulliautomatix.ics.uci.edu>
Gerrit-Reviewer: Murtadha Hubail <hu...@gmail.com>
Gerrit-Reviewer: Till Westmann <ti...@apache.org>
Gerrit-Reviewer: Yingyi Bu <bu...@gmail.com>
Gerrit-Reviewer: abdullah alamoudi <ba...@gmail.com>
Gerrit-HasComments: No

Change in asterixdb[master]: Introduce MessagingNetworkManager for NC2NC AppMessaging

Posted by "Jenkins (Code Review)" <do...@asterixdb.incubator.apache.org>.
Jenkins has posted comments on this change.

Change subject: Introduce MessagingNetworkManager for NC2NC AppMessaging
......................................................................


Patch Set 3:

Build Started https://asterix-jenkins.ics.uci.edu/job/asterix-gerrit-notopic/1529/

-- 
To view, visit https://asterix-gerrit.ics.uci.edu/897
To unsubscribe, visit https://asterix-gerrit.ics.uci.edu/settings

Gerrit-MessageType: comment
Gerrit-Change-Id: I5c0bd7c11c1e78954ebceff49cb274d8073a64bd
Gerrit-PatchSet: 3
Gerrit-Project: asterixdb
Gerrit-Branch: master
Gerrit-Owner: Murtadha Hubail <hu...@gmail.com>
Gerrit-Reviewer: Jenkins <je...@fulliautomatix.ics.uci.edu>
Gerrit-HasComments: No

Change in asterixdb[master]: Introduce MessagingNetworkManager for NC2NC AppMessaging

Posted by "Murtadha Hubail (Code Review)" <do...@asterixdb.incubator.apache.org>.
Hello Jenkins,

I'd like you to reexamine a change.  Please visit

    https://asterix-gerrit.ics.uci.edu/897

to look at the new patch set (#2).

Change subject: Introduce MessagingNetworkManager for NC2NC AppMessaging
......................................................................

Introduce MessagingNetworkManager for NC2NC AppMessaging

This change introduces MessagingNetworkManager to NodeControllerService.
The MessagingNetworkManager is used to open channels that are used for NC2NC
application messaging. The messaging channels are passed to the application
through the IMessageBroker APIs and should be managed by the application itself.

Change-Id: I5c0bd7c11c1e78954ebceff49cb274d8073a64bd
---
M asterixdb/asterix-app/src/main/java/org/apache/asterix/api/common/AsterixHyracksIntegrationUtil.java
M asterixdb/asterix-app/src/main/java/org/apache/asterix/messaging/NCMessageBroker.java
A asterixdb/asterix-app/src/main/java/org/apache/asterix/messaging/NCMessagingHelper.java
M asterixdb/asterix-common/src/main/java/org/apache/asterix/common/messaging/api/IApplicationMessage.java
M asterixdb/asterix-common/src/main/java/org/apache/asterix/common/messaging/api/INCMessageBroker.java
M asterixdb/asterix-transactions/src/main/java/org/apache/asterix/transaction/management/resource/GlobalResourceIdFactory.java
M hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/client/NodeControllerInfo.java
R hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/IBufferAcceptor.java
R hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/IBufferFactory.java
C hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/IChannelControlBlock.java
R hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/IChannelReadInterface.java
R hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/IChannelWriteInterface.java
R hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/ICloseableBufferAcceptor.java
M hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/messages/IMessageBroker.java
M hyracks-fullstack/hyracks/hyracks-comm/src/main/java/org/apache/hyracks/comm/channels/DatasetNetworkInputChannel.java
M hyracks-fullstack/hyracks/hyracks-comm/src/main/java/org/apache/hyracks/comm/channels/NetworkInputChannel.java
M hyracks-fullstack/hyracks/hyracks-comm/src/main/java/org/apache/hyracks/comm/channels/NetworkOutputChannel.java
M hyracks-fullstack/hyracks/hyracks-comm/src/main/java/org/apache/hyracks/comm/channels/ReadBufferFactory.java
M hyracks-fullstack/hyracks/hyracks-control/hyracks-control-cc/src/main/java/org/apache/hyracks/control/cc/NodeControllerState.java
M hyracks-fullstack/hyracks/hyracks-control/hyracks-control-cc/src/main/java/org/apache/hyracks/control/cc/work/GetNodeControllersInfoWork.java
M hyracks-fullstack/hyracks/hyracks-control/hyracks-control-common/src/main/java/org/apache/hyracks/control/common/controllers/NCConfig.java
M hyracks-fullstack/hyracks/hyracks-control/hyracks-control-common/src/main/java/org/apache/hyracks/control/common/controllers/NodeRegistration.java
M hyracks-fullstack/hyracks/hyracks-control/hyracks-control-nc/src/main/java/org/apache/hyracks/control/nc/NodeControllerService.java
M hyracks-fullstack/hyracks/hyracks-control/hyracks-control-nc/src/main/java/org/apache/hyracks/control/nc/net/DatasetNetworkManager.java
A hyracks-fullstack/hyracks/hyracks-control/hyracks-control-nc/src/main/java/org/apache/hyracks/control/nc/net/MessagingNetworkManager.java
M hyracks-fullstack/hyracks/hyracks-control/hyracks-control-nc/src/main/java/org/apache/hyracks/control/nc/net/NetworkManager.java
M hyracks-fullstack/hyracks/hyracks-hdfs/hyracks-hdfs-core/src/test/java/org/apache/hyracks/hdfs/scheduler/SchedulerTest.java
M hyracks-fullstack/hyracks/hyracks-hdfs/hyracks-hdfs-core/src/test/java/org/apache/hyracks/hdfs/utils/TestUtils.java
M hyracks-fullstack/hyracks/hyracks-hdfs/hyracks-hdfs-core/src/test/java/org/apache/hyracks/hdfs2/scheduler/SchedulerTest.java
M hyracks-fullstack/hyracks/hyracks-net/pom.xml
M hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/protocols/muxdemux/ChannelControlBlock.java
M hyracks-fullstack/hyracks/hyracks-net/src/test/java/org/apache/hyracks/net/tests/NetTest.java
32 files changed, 766 insertions(+), 200 deletions(-)


  git pull ssh://asterix-gerrit.ics.uci.edu:29418/asterixdb refs/changes/97/897/2
-- 
To view, visit https://asterix-gerrit.ics.uci.edu/897
To unsubscribe, visit https://asterix-gerrit.ics.uci.edu/settings

Gerrit-MessageType: newpatchset
Gerrit-Change-Id: I5c0bd7c11c1e78954ebceff49cb274d8073a64bd
Gerrit-PatchSet: 2
Gerrit-Project: asterixdb
Gerrit-Branch: master
Gerrit-Owner: Murtadha Hubail <hu...@gmail.com>
Gerrit-Reviewer: Jenkins <je...@fulliautomatix.ics.uci.edu>

Change in asterixdb[master]: Introduce MessagingNetworkManager for NC2NC AppMessaging

Posted by "Murtadha Hubail (Code Review)" <do...@asterixdb.incubator.apache.org>.
Hello Till Westmann, Jenkins,

I'd like you to reexamine a change.  Please visit

    https://asterix-gerrit.ics.uci.edu/897

to look at the new patch set (#8).

Change subject: Introduce MessagingNetworkManager for NC2NC AppMessaging
......................................................................

Introduce MessagingNetworkManager for NC2NC AppMessaging

This change introduces MessagingNetworkManager to NodeControllerService.
The MessagingNetworkManager is used to open channels that are used for NC2NC
application messaging. The read and write interfaces of the messaging channels
are set by a MessagingChannelInterfaceFactory which is set by the application.

Change-Id: I5c0bd7c11c1e78954ebceff49cb274d8073a64bd
---
M asterixdb/asterix-active/src/main/java/org/apache/asterix/active/ActiveManager.java
M asterixdb/asterix-app/src/main/java/org/apache/asterix/api/common/AsterixHyracksIntegrationUtil.java
M asterixdb/asterix-app/src/main/java/org/apache/asterix/app/nc/AsterixNCAppRuntimeContext.java
M asterixdb/asterix-app/src/main/java/org/apache/asterix/hyracks/bootstrap/NCApplicationEntryPoint.java
A asterixdb/asterix-app/src/main/java/org/apache/asterix/messaging/MessagingChannelInterfaceFactory.java
A asterixdb/asterix-app/src/main/java/org/apache/asterix/messaging/MessagingChannelReadInterface.java
A asterixdb/asterix-app/src/main/java/org/apache/asterix/messaging/MessagingChannelWriteInterface.java
M asterixdb/asterix-app/src/main/java/org/apache/asterix/messaging/NCMessageBroker.java
M asterixdb/asterix-app/src/main/resources/asterix-build-configuration.xml
M asterixdb/asterix-common/pom.xml
M asterixdb/asterix-common/src/main/java/org/apache/asterix/common/config/IAsterixPropertiesProvider.java
A asterixdb/asterix-common/src/main/java/org/apache/asterix/common/config/MessagingProperties.java
R asterixdb/asterix-common/src/main/java/org/apache/asterix/common/memory/ConcurrentFramePool.java
R asterixdb/asterix-common/src/main/java/org/apache/asterix/common/memory/FrameAction.java
M asterixdb/asterix-common/src/main/java/org/apache/asterix/common/messaging/api/IApplicationMessage.java
M asterixdb/asterix-common/src/main/java/org/apache/asterix/common/messaging/api/INCMessageBroker.java
R asterixdb/asterix-common/src/test/java/org/apache/asterix/test/memory/ConcurrentFramePoolUnitTest.java
M asterixdb/asterix-external-data/src/main/java/org/apache/asterix/external/feed/dataflow/FeedRuntimeInputHandler.java
M asterixdb/asterix-external-data/src/test/java/org/apache/asterix/external/feed/test/InputHandlerTest.java
M asterixdb/asterix-om/src/main/java/org/apache/asterix/om/util/AsterixAppContextInfo.java
M asterixdb/asterix-transactions/src/main/java/org/apache/asterix/transaction/management/resource/GlobalResourceIdFactory.java
M hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/application/INCApplicationContext.java
M hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/client/NodeControllerInfo.java
R hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/IBufferAcceptor.java
R hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/IBufferFactory.java
A hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/IChannelControlBlock.java
C hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/IChannelInterfaceFactory.java
R hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/IChannelReadInterface.java
R hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/IChannelWriteInterface.java
R hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/ICloseableBufferAcceptor.java
C hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/IConnectionWriterState.java
R hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/MuxDemuxCommand.java
R hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/exceptions/NetException.java
M hyracks-fullstack/hyracks/hyracks-client/src/main/java/org/apache/hyracks/client/net/ClientNetworkManager.java
M hyracks-fullstack/hyracks/hyracks-comm/src/main/java/org/apache/hyracks/comm/channels/DatasetNetworkInputChannel.java
M hyracks-fullstack/hyracks/hyracks-comm/src/main/java/org/apache/hyracks/comm/channels/IChannelConnectionFactory.java
M hyracks-fullstack/hyracks/hyracks-comm/src/main/java/org/apache/hyracks/comm/channels/NetworkInputChannel.java
M hyracks-fullstack/hyracks/hyracks-comm/src/main/java/org/apache/hyracks/comm/channels/NetworkOutputChannel.java
M hyracks-fullstack/hyracks/hyracks-comm/src/main/java/org/apache/hyracks/comm/channels/ReadBufferFactory.java
M hyracks-fullstack/hyracks/hyracks-control/hyracks-control-cc/src/main/java/org/apache/hyracks/control/cc/NodeControllerState.java
M hyracks-fullstack/hyracks/hyracks-control/hyracks-control-cc/src/main/java/org/apache/hyracks/control/cc/work/GetNodeControllersInfoWork.java
M hyracks-fullstack/hyracks/hyracks-control/hyracks-control-common/src/main/java/org/apache/hyracks/control/common/controllers/NCConfig.java
M hyracks-fullstack/hyracks/hyracks-control/hyracks-control-common/src/main/java/org/apache/hyracks/control/common/controllers/NodeRegistration.java
M hyracks-fullstack/hyracks/hyracks-control/hyracks-control-nc/src/main/java/org/apache/hyracks/control/nc/NCDriver.java
M hyracks-fullstack/hyracks/hyracks-control/hyracks-control-nc/src/main/java/org/apache/hyracks/control/nc/NodeControllerService.java
M hyracks-fullstack/hyracks/hyracks-control/hyracks-control-nc/src/main/java/org/apache/hyracks/control/nc/application/NCApplicationContext.java
M hyracks-fullstack/hyracks/hyracks-control/hyracks-control-nc/src/main/java/org/apache/hyracks/control/nc/net/DatasetNetworkManager.java
A hyracks-fullstack/hyracks/hyracks-control/hyracks-control-nc/src/main/java/org/apache/hyracks/control/nc/net/MessagingNetworkManager.java
M hyracks-fullstack/hyracks/hyracks-control/hyracks-control-nc/src/main/java/org/apache/hyracks/control/nc/net/NetworkManager.java
M hyracks-fullstack/hyracks/hyracks-hdfs/hyracks-hdfs-core/src/test/java/org/apache/hyracks/hdfs/scheduler/SchedulerTest.java
M hyracks-fullstack/hyracks/hyracks-hdfs/hyracks-hdfs-core/src/test/java/org/apache/hyracks/hdfs/utils/TestUtils.java
M hyracks-fullstack/hyracks/hyracks-hdfs/hyracks-hdfs-core/src/test/java/org/apache/hyracks/hdfs2/scheduler/SchedulerTest.java
M hyracks-fullstack/hyracks/hyracks-net/pom.xml
A hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/protocols/muxdemux/AbstractChannelReadInterface.java
A hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/protocols/muxdemux/AbstractChannelWriteInterface.java
M hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/protocols/muxdemux/ChannelControlBlock.java
M hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/protocols/muxdemux/ChannelSet.java
A hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/protocols/muxdemux/FullFrameChannelInterfaceFactory.java
A hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/protocols/muxdemux/FullFrameChannelReadInterface.java
A hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/protocols/muxdemux/FullFrameChannelWriteInterface.java
M hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/protocols/muxdemux/MultiplexedConnection.java
M hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/protocols/muxdemux/MuxDemux.java
M hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/protocols/tcp/ITCPConnectionEventListener.java
M hyracks-fullstack/hyracks/hyracks-net/src/test/java/org/apache/hyracks/net/tests/NetTest.java
M hyracks-fullstack/hyracks/hyracks-test-support/src/main/java/org/apache/hyracks/test/support/TestNCApplicationContext.java
65 files changed, 1,885 insertions(+), 669 deletions(-)


  git pull ssh://asterix-gerrit.ics.uci.edu:29418/asterixdb refs/changes/97/897/8
-- 
To view, visit https://asterix-gerrit.ics.uci.edu/897
To unsubscribe, visit https://asterix-gerrit.ics.uci.edu/settings

Gerrit-MessageType: newpatchset
Gerrit-Change-Id: I5c0bd7c11c1e78954ebceff49cb274d8073a64bd
Gerrit-PatchSet: 8
Gerrit-Project: asterixdb
Gerrit-Branch: master
Gerrit-Owner: Murtadha Hubail <hu...@gmail.com>
Gerrit-Reviewer: Jenkins <je...@fulliautomatix.ics.uci.edu>
Gerrit-Reviewer: Murtadha Hubail <hu...@gmail.com>
Gerrit-Reviewer: Till Westmann <ti...@apache.org>
Gerrit-Reviewer: Yingyi Bu <bu...@gmail.com>
Gerrit-Reviewer: abdullah alamoudi <ba...@gmail.com>

Change in asterixdb[master]: Introduce MessagingNetworkManager for NC2NC AppMessaging

Posted by "Jenkins (Code Review)" <do...@asterixdb.incubator.apache.org>.
Jenkins has posted comments on this change.

Change subject: Introduce MessagingNetworkManager for NC2NC AppMessaging
......................................................................


Patch Set 8:

Integration Tests Started https://asterix-jenkins.ics.uci.edu/job/asterix-gerrit-integration-tests/465/

-- 
To view, visit https://asterix-gerrit.ics.uci.edu/897
To unsubscribe, visit https://asterix-gerrit.ics.uci.edu/settings

Gerrit-MessageType: comment
Gerrit-Change-Id: I5c0bd7c11c1e78954ebceff49cb274d8073a64bd
Gerrit-PatchSet: 8
Gerrit-Project: asterixdb
Gerrit-Branch: master
Gerrit-Owner: Murtadha Hubail <hu...@gmail.com>
Gerrit-Reviewer: Jenkins <je...@fulliautomatix.ics.uci.edu>
Gerrit-Reviewer: Murtadha Hubail <hu...@gmail.com>
Gerrit-Reviewer: Till Westmann <ti...@apache.org>
Gerrit-Reviewer: Yingyi Bu <bu...@gmail.com>
Gerrit-Reviewer: abdullah alamoudi <ba...@gmail.com>
Gerrit-HasComments: No

Change in asterixdb[master]: Introduce MessagingNetworkManager for NC2NC AppMessaging

Posted by "Jenkins (Code Review)" <do...@asterixdb.incubator.apache.org>.
Jenkins has posted comments on this change.

Change subject: Introduce MessagingNetworkManager for NC2NC AppMessaging
......................................................................


Patch Set 7:

Integration Tests Started https://asterix-jenkins.ics.uci.edu/job/asterix-gerrit-integration-tests/19/

-- 
To view, visit https://asterix-gerrit.ics.uci.edu/897
To unsubscribe, visit https://asterix-gerrit.ics.uci.edu/settings

Gerrit-MessageType: comment
Gerrit-Change-Id: I5c0bd7c11c1e78954ebceff49cb274d8073a64bd
Gerrit-PatchSet: 7
Gerrit-Project: asterixdb
Gerrit-Branch: master
Gerrit-Owner: Murtadha Hubail <hu...@gmail.com>
Gerrit-Reviewer: Jenkins <je...@fulliautomatix.ics.uci.edu>
Gerrit-Reviewer: Murtadha Hubail <hu...@gmail.com>
Gerrit-Reviewer: Till Westmann <ti...@apache.org>
Gerrit-Reviewer: Yingyi Bu <bu...@gmail.com>
Gerrit-Reviewer: abdullah alamoudi <ba...@gmail.com>
Gerrit-HasComments: No

Change in asterixdb[master]: Introduce MessagingNetworkManager for NC2NC AppMessaging

Posted by "Jenkins (Code Review)" <do...@asterixdb.incubator.apache.org>.
Jenkins has posted comments on this change.

Change subject: Introduce MessagingNetworkManager for NC2NC AppMessaging
......................................................................


Patch Set 7:

Integration Tests Started https://asterix-jenkins.ics.uci.edu/job/asterix-gerrit-integration-tests/29/

-- 
To view, visit https://asterix-gerrit.ics.uci.edu/897
To unsubscribe, visit https://asterix-gerrit.ics.uci.edu/settings

Gerrit-MessageType: comment
Gerrit-Change-Id: I5c0bd7c11c1e78954ebceff49cb274d8073a64bd
Gerrit-PatchSet: 7
Gerrit-Project: asterixdb
Gerrit-Branch: master
Gerrit-Owner: Murtadha Hubail <hu...@gmail.com>
Gerrit-Reviewer: Jenkins <je...@fulliautomatix.ics.uci.edu>
Gerrit-Reviewer: Murtadha Hubail <hu...@gmail.com>
Gerrit-Reviewer: Till Westmann <ti...@apache.org>
Gerrit-Reviewer: Yingyi Bu <bu...@gmail.com>
Gerrit-Reviewer: abdullah alamoudi <ba...@gmail.com>
Gerrit-HasComments: No

Change in asterixdb[master]: Introduce MessagingNetworkManager for NC2NC AppMessaging

Posted by "Jenkins (Code Review)" <do...@asterixdb.incubator.apache.org>.
Jenkins has posted comments on this change.

Change subject: Introduce MessagingNetworkManager for NC2NC AppMessaging
......................................................................


Patch Set 1:

Build Started https://asterix-jenkins.ics.uci.edu/job/asterix-gerrit-notopic/1527/

-- 
To view, visit https://asterix-gerrit.ics.uci.edu/897
To unsubscribe, visit https://asterix-gerrit.ics.uci.edu/settings

Gerrit-MessageType: comment
Gerrit-Change-Id: I5c0bd7c11c1e78954ebceff49cb274d8073a64bd
Gerrit-PatchSet: 1
Gerrit-Project: asterixdb
Gerrit-Branch: master
Gerrit-Owner: Murtadha Hubail <hu...@gmail.com>
Gerrit-Reviewer: Jenkins <je...@fulliautomatix.ics.uci.edu>
Gerrit-HasComments: No

Change in asterixdb[master]: Introduce MessagingNetworkManager for NC2NC AppMessaging

Posted by "Jenkins (Code Review)" <do...@asterixdb.incubator.apache.org>.
Jenkins has posted comments on this change.

Change subject: Introduce MessagingNetworkManager for NC2NC AppMessaging
......................................................................


Patch Set 7: Integration-Tests+1

Integration Tests Successful

https://asterix-jenkins.ics.uci.edu/job/asterix-gerrit-integration-tests/29/ : SUCCESS

-- 
To view, visit https://asterix-gerrit.ics.uci.edu/897
To unsubscribe, visit https://asterix-gerrit.ics.uci.edu/settings

Gerrit-MessageType: comment
Gerrit-Change-Id: I5c0bd7c11c1e78954ebceff49cb274d8073a64bd
Gerrit-PatchSet: 7
Gerrit-Project: asterixdb
Gerrit-Branch: master
Gerrit-Owner: Murtadha Hubail <hu...@gmail.com>
Gerrit-Reviewer: Jenkins <je...@fulliautomatix.ics.uci.edu>
Gerrit-Reviewer: Murtadha Hubail <hu...@gmail.com>
Gerrit-Reviewer: Till Westmann <ti...@apache.org>
Gerrit-Reviewer: Yingyi Bu <bu...@gmail.com>
Gerrit-Reviewer: abdullah alamoudi <ba...@gmail.com>
Gerrit-HasComments: No

Change in asterixdb[master]: Introduce MessagingNetworkManager for NC2NC AppMessaging

Posted by "Jenkins (Code Review)" <do...@asterixdb.incubator.apache.org>.
Jenkins has posted comments on this change.

Change subject: Introduce MessagingNetworkManager for NC2NC AppMessaging
......................................................................


Patch Set 7:

WARNING: THIS CHANGE CONTAINS CROSS-PRODUCT CHANGES IN:
* asterixdb
* hyracks-fullstack

PLEASE REVIEW CAREFULLY AND LOOK FOR API CHANGES!

-- 
To view, visit https://asterix-gerrit.ics.uci.edu/897
To unsubscribe, visit https://asterix-gerrit.ics.uci.edu/settings

Gerrit-MessageType: comment
Gerrit-Change-Id: I5c0bd7c11c1e78954ebceff49cb274d8073a64bd
Gerrit-PatchSet: 7
Gerrit-Project: asterixdb
Gerrit-Branch: master
Gerrit-Owner: Murtadha Hubail <hu...@gmail.com>
Gerrit-Reviewer: Jenkins <je...@fulliautomatix.ics.uci.edu>
Gerrit-Reviewer: Murtadha Hubail <hu...@gmail.com>
Gerrit-Reviewer: Till Westmann <ti...@apache.org>
Gerrit-Reviewer: Yingyi Bu <bu...@gmail.com>
Gerrit-Reviewer: abdullah alamoudi <ba...@gmail.com>
Gerrit-HasComments: No

Change in asterixdb[master]: Introduce MessagingNetworkManager for NC2NC AppMessaging

Posted by "Murtadha Hubail (Code Review)" <do...@asterixdb.incubator.apache.org>.
Hello Jenkins,

I'd like you to reexamine a change.  Please visit

    https://asterix-gerrit.ics.uci.edu/897

to look at the new patch set (#6).

Change subject: Introduce MessagingNetworkManager for NC2NC AppMessaging
......................................................................

Introduce MessagingNetworkManager for NC2NC AppMessaging

This change introduces MessagingNetworkManager to NodeControllerService.
The MessagingNetworkManager is used to open channels that are used for NC2NC
application messaging. The read and write interfaces of the messaging channels
are set by a MessagingChannelInterfaceFactory which is set by the application.

Change-Id: I5c0bd7c11c1e78954ebceff49cb274d8073a64bd
---
M asterixdb/asterix-app/src/main/java/org/apache/asterix/api/common/AsterixAppRuntimeContext.java
M asterixdb/asterix-app/src/main/java/org/apache/asterix/api/common/AsterixHyracksIntegrationUtil.java
M asterixdb/asterix-app/src/main/java/org/apache/asterix/hyracks/bootstrap/NCApplicationEntryPoint.java
A asterixdb/asterix-app/src/main/java/org/apache/asterix/messaging/MessagingChannelInterfaceFactory.java
A asterixdb/asterix-app/src/main/java/org/apache/asterix/messaging/MessagingChannelReadInterface.java
A asterixdb/asterix-app/src/main/java/org/apache/asterix/messaging/MessagingChannelWriteInterface.java
M asterixdb/asterix-app/src/main/java/org/apache/asterix/messaging/NCMessageBroker.java
M asterixdb/asterix-app/src/main/resources/asterix-build-configuration.xml
M asterixdb/asterix-common/src/main/java/org/apache/asterix/common/config/IAsterixPropertiesProvider.java
A asterixdb/asterix-common/src/main/java/org/apache/asterix/common/config/MessagingProperties.java
R asterixdb/asterix-common/src/main/java/org/apache/asterix/common/memory/ConcurrentFramePool.java
R asterixdb/asterix-common/src/main/java/org/apache/asterix/common/memory/FrameAction.java
M asterixdb/asterix-common/src/main/java/org/apache/asterix/common/messaging/api/IApplicationMessage.java
M asterixdb/asterix-common/src/main/java/org/apache/asterix/common/messaging/api/INCMessageBroker.java
R asterixdb/asterix-common/src/test/java/org/apache/asterix/test/memory/ConcurrentFramePoolUnitTest.java
M asterixdb/asterix-external-data/src/main/java/org/apache/asterix/external/feed/dataflow/FeedRuntimeInputHandler.java
M asterixdb/asterix-external-data/src/main/java/org/apache/asterix/external/feed/management/FeedManager.java
M asterixdb/asterix-external-data/src/test/java/org/apache/asterix/external/feed/test/InputHandlerTest.java
M asterixdb/asterix-om/src/main/java/org/apache/asterix/om/util/AsterixAppContextInfo.java
M asterixdb/asterix-transactions/src/main/java/org/apache/asterix/transaction/management/resource/GlobalResourceIdFactory.java
M hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/application/INCApplicationContext.java
M hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/client/NodeControllerInfo.java
R hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/IBufferAcceptor.java
R hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/IBufferFactory.java
A hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/IChannelControlBlock.java
C hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/IChannelInterfaceFactory.java
R hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/IChannelReadInterface.java
R hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/IChannelWriteInterface.java
R hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/ICloseableBufferAcceptor.java
C hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/IConnectionWriterState.java
R hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/comm/MuxDemuxCommand.java
R hyracks-fullstack/hyracks/hyracks-api/src/main/java/org/apache/hyracks/api/exceptions/NetException.java
M hyracks-fullstack/hyracks/hyracks-client/src/main/java/org/apache/hyracks/client/net/ClientNetworkManager.java
M hyracks-fullstack/hyracks/hyracks-comm/src/main/java/org/apache/hyracks/comm/channels/DatasetNetworkInputChannel.java
M hyracks-fullstack/hyracks/hyracks-comm/src/main/java/org/apache/hyracks/comm/channels/IChannelConnectionFactory.java
M hyracks-fullstack/hyracks/hyracks-comm/src/main/java/org/apache/hyracks/comm/channels/NetworkInputChannel.java
M hyracks-fullstack/hyracks/hyracks-comm/src/main/java/org/apache/hyracks/comm/channels/NetworkOutputChannel.java
M hyracks-fullstack/hyracks/hyracks-comm/src/main/java/org/apache/hyracks/comm/channels/ReadBufferFactory.java
M hyracks-fullstack/hyracks/hyracks-control/hyracks-control-cc/src/main/java/org/apache/hyracks/control/cc/NodeControllerState.java
M hyracks-fullstack/hyracks/hyracks-control/hyracks-control-cc/src/main/java/org/apache/hyracks/control/cc/work/GetNodeControllersInfoWork.java
M hyracks-fullstack/hyracks/hyracks-control/hyracks-control-common/src/main/java/org/apache/hyracks/control/common/controllers/NCConfig.java
M hyracks-fullstack/hyracks/hyracks-control/hyracks-control-common/src/main/java/org/apache/hyracks/control/common/controllers/NodeRegistration.java
M hyracks-fullstack/hyracks/hyracks-control/hyracks-control-nc/src/main/java/org/apache/hyracks/control/nc/NodeControllerService.java
M hyracks-fullstack/hyracks/hyracks-control/hyracks-control-nc/src/main/java/org/apache/hyracks/control/nc/application/NCApplicationContext.java
M hyracks-fullstack/hyracks/hyracks-control/hyracks-control-nc/src/main/java/org/apache/hyracks/control/nc/net/DatasetNetworkManager.java
A hyracks-fullstack/hyracks/hyracks-control/hyracks-control-nc/src/main/java/org/apache/hyracks/control/nc/net/MessagingNetworkManager.java
M hyracks-fullstack/hyracks/hyracks-control/hyracks-control-nc/src/main/java/org/apache/hyracks/control/nc/net/NetworkManager.java
M hyracks-fullstack/hyracks/hyracks-hdfs/hyracks-hdfs-core/src/test/java/org/apache/hyracks/hdfs/scheduler/SchedulerTest.java
M hyracks-fullstack/hyracks/hyracks-hdfs/hyracks-hdfs-core/src/test/java/org/apache/hyracks/hdfs/utils/TestUtils.java
M hyracks-fullstack/hyracks/hyracks-hdfs/hyracks-hdfs-core/src/test/java/org/apache/hyracks/hdfs2/scheduler/SchedulerTest.java
M hyracks-fullstack/hyracks/hyracks-net/pom.xml
M hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/protocols/muxdemux/ChannelControlBlock.java
M hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/protocols/muxdemux/ChannelSet.java
A hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/protocols/muxdemux/FullFrameChannelInterfaceFactory.java
A hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/protocols/muxdemux/FullFrameChannelReadInterface.java
A hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/protocols/muxdemux/FullFrameChannelWriteInterface.java
M hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/protocols/muxdemux/MultiplexedConnection.java
M hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/protocols/muxdemux/MuxDemux.java
M hyracks-fullstack/hyracks/hyracks-net/src/main/java/org/apache/hyracks/net/protocols/tcp/ITCPConnectionEventListener.java
M hyracks-fullstack/hyracks/hyracks-net/src/test/java/org/apache/hyracks/net/tests/NetTest.java
M hyracks-fullstack/hyracks/hyracks-server/src/main/java/org/apache/hyracks/server/drivers/VirtualClusterDriver.java
M hyracks-fullstack/hyracks/hyracks-test-support/src/main/java/org/apache/hyracks/test/support/TestNCApplicationContext.java
62 files changed, 1,920 insertions(+), 583 deletions(-)


  git pull ssh://asterix-gerrit.ics.uci.edu:29418/asterixdb refs/changes/97/897/6
-- 
To view, visit https://asterix-gerrit.ics.uci.edu/897
To unsubscribe, visit https://asterix-gerrit.ics.uci.edu/settings

Gerrit-MessageType: newpatchset
Gerrit-Change-Id: I5c0bd7c11c1e78954ebceff49cb274d8073a64bd
Gerrit-PatchSet: 6
Gerrit-Project: asterixdb
Gerrit-Branch: master
Gerrit-Owner: Murtadha Hubail <hu...@gmail.com>
Gerrit-Reviewer: Jenkins <je...@fulliautomatix.ics.uci.edu>
Gerrit-Reviewer: Till Westmann <ti...@apache.org>
Gerrit-Reviewer: Yingyi Bu <bu...@gmail.com>
Gerrit-Reviewer: abdullah alamoudi <ba...@gmail.com>

Change in asterixdb[master]: Introduce MessagingNetworkManager for NC2NC AppMessaging

Posted by "Jenkins (Code Review)" <do...@asterixdb.incubator.apache.org>.
Jenkins has posted comments on this change.

Change subject: Introduce MessagingNetworkManager for NC2NC AppMessaging
......................................................................


Patch Set 8:

WARNING: THIS CHANGE CONTAINS CROSS-PRODUCT CHANGES IN:
* asterixdb
* hyracks-fullstack

PLEASE REVIEW CAREFULLY AND LOOK FOR API CHANGES!

-- 
To view, visit https://asterix-gerrit.ics.uci.edu/897
To unsubscribe, visit https://asterix-gerrit.ics.uci.edu/settings

Gerrit-MessageType: comment
Gerrit-Change-Id: I5c0bd7c11c1e78954ebceff49cb274d8073a64bd
Gerrit-PatchSet: 8
Gerrit-Project: asterixdb
Gerrit-Branch: master
Gerrit-Owner: Murtadha Hubail <hu...@gmail.com>
Gerrit-Reviewer: Jenkins <je...@fulliautomatix.ics.uci.edu>
Gerrit-Reviewer: Murtadha Hubail <hu...@gmail.com>
Gerrit-Reviewer: Till Westmann <ti...@apache.org>
Gerrit-Reviewer: Yingyi Bu <bu...@gmail.com>
Gerrit-Reviewer: abdullah alamoudi <ba...@gmail.com>
Gerrit-HasComments: No