[GitHub] [phoenix-omid] joshelser commented on a change in pull request #92: OMID-202 Refactor Omid to use Netty 4

2021-03-23 Thread GitBox


joshelser commented on a change in pull request #92:
URL: https://github.com/apache/phoenix-omid/pull/92#discussion_r599858949



##
File path: 
transaction-client/src/main/java/org/apache/omid/tso/client/TSOClient.java
##
@@ -1012,37 +1016,34 @@ private void closeChannelAndErrorRequests() {
 }
 
 @Override
-public void channelConnected(ChannelHandlerContext ctx, 
ChannelStateEvent e) {
-currentChannel = e.getChannel();
-LOG.debug("HANDLER (CHANNEL CONNECTED): Connection {}. Sending 
connected event to FSM", e);
-fsm.sendEvent(new ConnectedEvent(e.getChannel()));
+public void channelActive(ChannelHandlerContext ctx) {
+currentChannel = ctx.channel();
+LOG.debug("HANDLER (CHANNEL ACTIVE): Connection {}. Sending 
connected event to FSM", ctx.channel());
+fsm.sendEvent(new ConnectedEvent(ctx.channel()));
 }
 
 @Override
-public void channelDisconnected(ChannelHandlerContext ctx, 
ChannelStateEvent e) throws Exception {
-LOG.debug("HANDLER (CHANNEL DISCONNECTED): Connection {}. Sending 
error event to FSM", e);
+public void channelInactive(ChannelHandlerContext ctx) throws 
Exception {
+LOG.debug("HANDLER (CHANNEL INACTIVE): Connection {}. Sending 
error, then channelClosed event to FSM", ctx.channel());
+// Netty 3 had separate callbacks, and the FSM expects both events.
+// Sending both is much easier than rewriting the FSM

Review comment:
   Fine to come back to it later!




-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

For queries about this service, please contact Infrastructure at:
us...@infra.apache.org




[GitHub] [phoenix-omid] joshelser commented on a change in pull request #92: OMID-202 Refactor Omid to use Netty 4

2021-03-23 Thread GitBox


joshelser commented on a change in pull request #92:
URL: https://github.com/apache/phoenix-omid/pull/92#discussion_r599858525



##
File path: 
transaction-client/src/main/java/org/apache/omid/tso/client/TSOClient.java
##
@@ -342,7 +344,7 @@ public void nodeChanged() throws Exception {
 setTSOAddress(hp.getHost(), hp.getPort());
 epoch = Long.parseLong(currentTSOAndEpochArray[1]);
 LOG.info("CurrentTSO ZNode changed. New TSO Host & Port {}/Epoch {}", 
hp, getEpoch());
-if (currentChannel != null && currentChannel.isConnected()) {
+if (currentChannel != null && currentChannel.isActive()) {

Review comment:
   Ah, sorry, I was suggesting to consolidate these two things into one 
method, aka `boolean isChannelActive(Channel) { return channel != null && 
channel.isActive();}`
   
   Can ignore if you don't think there's value in it :)




-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

For queries about this service, please contact Infrastructure at:
us...@infra.apache.org




[GitHub] [phoenix-omid] joshelser commented on a change in pull request #92: OMID-202 Refactor Omid to use Netty 4

2021-03-19 Thread GitBox


joshelser commented on a change in pull request #92:
URL: https://github.com/apache/phoenix-omid/pull/92#discussion_r597882282



##
File path: 
transaction-client/src/main/java/org/apache/omid/tso/client/TSOClient.java
##
@@ -342,7 +344,7 @@ public void nodeChanged() throws Exception {
 setTSOAddress(hp.getHost(), hp.getPort());
 epoch = Long.parseLong(currentTSOAndEpochArray[1]);
 LOG.info("CurrentTSO ZNode changed. New TSO Host & Port {}/Epoch {}", 
hp, getEpoch());
-if (currentChannel != null && currentChannel.isConnected()) {
+if (currentChannel != null && currentChannel.isActive()) {

Review comment:
   Might be nice to consolidate this into a helper since you're modifying 
it.

##
File path: 
transaction-client/src/main/java/org/apache/omid/tso/client/TSOClient.java
##
@@ -513,7 +515,7 @@ void decrementRetries() {
 }
 
 public StateMachine.State handleEvent(CloseEvent e) {
-factory.releaseExternalResources();
+bootstrap.config().group().shutdownGracefully();

Review comment:
   If you have two racing `CloseEvent` messages, does 
`shutdownGracefully()` handle this correctly (i.e. is synchronized and exits 
quietly on the 2nd).

##
File path: 
transaction-client/src/main/java/org/apache/omid/tso/client/TSOClient.java
##
@@ -1012,37 +1016,34 @@ private void closeChannelAndErrorRequests() {
 }
 
 @Override
-public void channelConnected(ChannelHandlerContext ctx, 
ChannelStateEvent e) {
-currentChannel = e.getChannel();
-LOG.debug("HANDLER (CHANNEL CONNECTED): Connection {}. Sending 
connected event to FSM", e);
-fsm.sendEvent(new ConnectedEvent(e.getChannel()));
+public void channelActive(ChannelHandlerContext ctx) {
+currentChannel = ctx.channel();
+LOG.debug("HANDLER (CHANNEL ACTIVE): Connection {}. Sending 
connected event to FSM", ctx.channel());
+fsm.sendEvent(new ConnectedEvent(ctx.channel()));
 }
 
 @Override
-public void channelDisconnected(ChannelHandlerContext ctx, 
ChannelStateEvent e) throws Exception {
-LOG.debug("HANDLER (CHANNEL DISCONNECTED): Connection {}. Sending 
error event to FSM", e);
+public void channelInactive(ChannelHandlerContext ctx) throws 
Exception {
+LOG.debug("HANDLER (CHANNEL INACTIVE): Connection {}. Sending 
error, then channelClosed event to FSM", ctx.channel());
+// Netty 3 had separate callbacks, and the FSM expects both events.
+// Sending both is much easier than rewriting the FSM

Review comment:
   No issues if we send both of these events back? Still looking to see 
where those are consumed.

##
File path: tso-server/src/test/java/org/apache/omid/tso/client/TSOClientRaw.java
##
@@ -58,39 +62,39 @@
 private final Channel channel;
 
 public TSOClientRaw(String host, int port) throws InterruptedException, 
ExecutionException {
-// Start client with Nb of active threads = 3 as maximum.
-ChannelFactory factory = new NioClientSocketChannelFactory(
-Executors.newCachedThreadPool(
-new 
ThreadFactoryBuilder().setNameFormat("tsoclient-boss-%d").build()),
-Executors.newCachedThreadPool(
-new 
ThreadFactoryBuilder().setNameFormat("tsoclient-worker-%d").build()), 3);
-// Create the bootstrap
-ClientBootstrap bootstrap = new ClientBootstrap(factory);
 
 InetSocketAddress addr = new InetSocketAddress(host, port);
 
-ChannelPipeline pipeline = bootstrap.getPipeline();
-pipeline.addLast("lengthbaseddecoder",
-new LengthFieldBasedFrameDecoder(8 * 1024, 0, 4, 0, 4));
-pipeline.addLast("lengthprepender", new LengthFieldPrepender(4));
-pipeline.addLast("protobufdecoder",
-new ProtobufDecoder(TSOProto.Response.getDefaultInstance()));
-pipeline.addLast("protobufencoder", new ProtobufEncoder());
-
-Handler handler = new Handler();
-pipeline.addLast("handler", handler);
-
-bootstrap.setOption("tcpNoDelay", true);
-bootstrap.setOption("keepAlive", true);
-bootstrap.setOption("reuseAddress", true);
-bootstrap.setOption("connectTimeoutMillis", 100);
+// Start client with Nb of active threads = 3
+ThreadFactory workerThreadFactory = new 
ThreadFactoryBuilder().setNameFormat("tsoclient-worker-%d").build();
+EventLoopGroup workerGroup = new NioEventLoopGroup(3, 
workerThreadFactory);

Review comment:
   The old code appears to have had separate boss and worker thread pools. 
Was the old boss pool unused?

##
File path: tso-server/src/main/java/org/apache/omid/tso/TSOChannelHandler.java
##
@@ -17,95 +17,116 @@
  */
 package org.apache.omid.tso;
 
-import