You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@ratis.apache.org by ru...@apache.org on 2020/11/06 02:37:14 UTC

[incubator-ratis] branch master updated: RATIS-1132. Primary and peer should use the same streamId (#257)

This is an automated email from the ASF dual-hosted git repository.

runzhiwang pushed a commit to branch master
in repository https://gitbox.apache.org/repos/asf/incubator-ratis.git


The following commit(s) were added to refs/heads/master by this push:
     new f0460f8  RATIS-1132. Primary and peer should use the same streamId (#257)
f0460f8 is described below

commit f0460f8e0b9c2041251b9199ef3d4bfd82488bcf
Author: runzhiwang <51...@users.noreply.github.com>
AuthorDate: Fri Nov 6 10:36:35 2020 +0800

    RATIS-1132. Primary and peer should use the same streamId (#257)
    
    * RATIS-1132. Primary and peer should use the same streamId
    
    * add unit test
    
    * fix code review
---
 .../org/apache/ratis/client/DataStreamClient.java  |  5 ++--
 .../org/apache/ratis/client/DataStreamRpcApi.java  | 29 ++++++++++++++++++++++
 .../ratis/client/impl/DataStreamClientImpl.java    | 10 +++++++-
 .../ratis/netty/server/NettyServerStreamRpc.java   | 15 +++++------
 .../ratis/datastream/DataStreamBaseTest.java       | 15 +++++++----
 .../ratis/datastream/TestDataStreamNetty.java      |  2 +-
 6 files changed, 59 insertions(+), 17 deletions(-)

diff --git a/ratis-client/src/main/java/org/apache/ratis/client/DataStreamClient.java b/ratis-client/src/main/java/org/apache/ratis/client/DataStreamClient.java
index 3d0c327..0bc0afc 100644
--- a/ratis-client/src/main/java/org/apache/ratis/client/DataStreamClient.java
+++ b/ratis-client/src/main/java/org/apache/ratis/client/DataStreamClient.java
@@ -17,7 +17,6 @@
  */
 package org.apache.ratis.client;
 
-import org.apache.ratis.client.api.DataStreamApi;
 import org.apache.ratis.client.impl.DataStreamClientImpl;
 import org.apache.ratis.conf.Parameters;
 import org.apache.ratis.conf.RaftProperties;
@@ -30,9 +29,9 @@ import org.slf4j.LoggerFactory;
 import java.io.Closeable;
 
 /**
- * A user interface extending {@link DataStreamApi}.
+ * A user interface extending {@link DataStreamRpcApi}.
  */
-public interface DataStreamClient extends DataStreamApi, Closeable {
+public interface DataStreamClient extends DataStreamRpcApi, Closeable {
   Logger LOG = LoggerFactory.getLogger(DataStreamClient.class);
 
   /** Return the rpc client instance **/
diff --git a/ratis-client/src/main/java/org/apache/ratis/client/DataStreamRpcApi.java b/ratis-client/src/main/java/org/apache/ratis/client/DataStreamRpcApi.java
new file mode 100644
index 0000000..409e4e1
--- /dev/null
+++ b/ratis-client/src/main/java/org/apache/ratis/client/DataStreamRpcApi.java
@@ -0,0 +1,29 @@
+/*
+ * 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.ratis.client;
+
+import org.apache.ratis.client.api.DataStreamApi;
+import org.apache.ratis.client.api.DataStreamOutput;
+import org.apache.ratis.protocol.RaftGroupId;
+
+/** An RPC interface which extends the user interface {@link DataStreamApi}. */
+public interface DataStreamRpcApi extends DataStreamApi {
+  /** Create a stream for primary server to send data to peer server. */
+  DataStreamOutput stream(RaftGroupId groupId, long streamId);
+}
diff --git a/ratis-client/src/main/java/org/apache/ratis/client/impl/DataStreamClientImpl.java b/ratis-client/src/main/java/org/apache/ratis/client/impl/DataStreamClientImpl.java
index 6d17df9..9efe56c 100644
--- a/ratis-client/src/main/java/org/apache/ratis/client/impl/DataStreamClientImpl.java
+++ b/ratis-client/src/main/java/org/apache/ratis/client/impl/DataStreamClientImpl.java
@@ -70,7 +70,10 @@ public class DataStreamClientImpl implements DataStreamClient {
     private long streamOffset = 0;
 
     public DataStreamOutputImpl(RaftGroupId groupId) {
-      final long streamId = RaftClientImpl.nextCallId();
+      this(groupId, RaftClientImpl.nextCallId());
+    }
+
+    public DataStreamOutputImpl(RaftGroupId groupId, long streamId) {
       this.header = new RaftClientRequest(clientId, raftServer.getId(), groupId, streamId,
           RaftClientRequest.writeRequestType());
       this.headerFuture = orderedStreamAsync.sendRequest(streamId, -1,
@@ -134,6 +137,11 @@ public class DataStreamClientImpl implements DataStreamClient {
   }
 
   @Override
+  public DataStreamOutputRpc stream(RaftGroupId gid, long streamId) {
+    return new DataStreamOutputImpl(gid, streamId);
+  }
+
+  @Override
   public void close() throws IOException {
     dataStreamClientRpc.close();
   }
diff --git a/ratis-netty/src/main/java/org/apache/ratis/netty/server/NettyServerStreamRpc.java b/ratis-netty/src/main/java/org/apache/ratis/netty/server/NettyServerStreamRpc.java
index adcd373..f983684 100644
--- a/ratis-netty/src/main/java/org/apache/ratis/netty/server/NettyServerStreamRpc.java
+++ b/ratis-netty/src/main/java/org/apache/ratis/netty/server/NettyServerStreamRpc.java
@@ -96,10 +96,10 @@ public class NettyServerStreamRpc implements DataStreamServerRpc {
       peers.addAll(newPeers);
     }
 
-    List<DataStreamOutputRpc> getDataStreamOutput(RaftGroupId groupId) throws IOException {
+    List<DataStreamOutputRpc> getDataStreamOutput(RaftGroupId groupId, long streamId) throws IOException {
       final List<DataStreamOutputRpc> outs = new ArrayList<>();
       try {
-        getDataStreamOutput(outs, groupId);
+        getDataStreamOutput(outs, groupId, streamId);
       } catch (IOException e) {
         outs.forEach(CloseAsync::closeAsync);
         throw e;
@@ -107,10 +107,11 @@ public class NettyServerStreamRpc implements DataStreamServerRpc {
       return outs;
     }
 
-    private void getDataStreamOutput(List<DataStreamOutputRpc> outs, RaftGroupId groupId) throws IOException {
+    private void getDataStreamOutput(List<DataStreamOutputRpc> outs, RaftGroupId groupId, long streamId)
+        throws IOException {
       for (RaftPeer peer : peers) {
         try {
-          outs.add((DataStreamOutputRpc) map.getProxy(peer.getId()).stream(groupId));
+          outs.add((DataStreamOutputRpc) map.getProxy(peer.getId()).stream(groupId, streamId));
         } catch (IOException e) {
           throw new IOException(map.getName() + ": Failed to getDataStreamOutput for " + peer, e);
         }
@@ -247,13 +248,13 @@ public class NettyServerStreamRpc implements DataStreamServerRpc {
     proxies.addPeers(newPeers);
   }
 
-  private StreamInfo newStreamInfo(ByteBuf buf) {
+  private StreamInfo newStreamInfo(ByteBuf buf, long streamId) {
     try {
       final RaftClientRequest request = ClientProtoUtils.toRaftClientRequest(
           RaftClientRequestProto.parseFrom(buf.nioBuffer()));
       final StateMachine stateMachine = server.getStateMachine(request.getRaftGroupId());
       return new StreamInfo(request, stateMachine.data().stream(request),
-          proxies.getDataStreamOutput(request.getRaftGroupId()));
+          proxies.getDataStreamOutput(request.getRaftGroupId(), streamId));
     } catch (Throwable e) {
       throw new CompletionException(e);
     }
@@ -364,7 +365,7 @@ public class NettyServerStreamRpc implements DataStreamServerRpc {
     final List<CompletableFuture<DataStreamReply>> remoteWrites = new ArrayList<>();
     final StreamMap.Key key = new StreamMap.Key(ctx.channel().id(), request.getStreamId());
     if (request.getType() == Type.STREAM_HEADER) {
-      info = streams.computeIfAbsent(key, id -> newStreamInfo(buf));
+      info = streams.computeIfAbsent(key, id -> newStreamInfo(buf, request.getStreamId()));
       localWrite = CompletableFuture.completedFuture(0L);
       for (DataStreamOutputRpc out : info.getDataStreamOutputs()) {
         remoteWrites.add(out.getHeaderFuture());
diff --git a/ratis-test/src/test/java/org/apache/ratis/datastream/DataStreamBaseTest.java b/ratis-test/src/test/java/org/apache/ratis/datastream/DataStreamBaseTest.java
index 18753d1..668cb17 100644
--- a/ratis-test/src/test/java/org/apache/ratis/datastream/DataStreamBaseTest.java
+++ b/ratis-test/src/test/java/org/apache/ratis/datastream/DataStreamBaseTest.java
@@ -21,7 +21,6 @@ import org.apache.ratis.BaseTest;
 import org.apache.ratis.MiniRaftCluster;
 import org.apache.ratis.client.RaftClient;
 import org.apache.ratis.client.impl.ClientProtoUtils;
-import org.apache.ratis.client.impl.DataStreamClientImpl;
 import org.apache.ratis.client.impl.DataStreamClientImpl.DataStreamOutputImpl;
 import org.apache.ratis.conf.RaftProperties;
 import org.apache.ratis.datastream.impl.DataStreamReplyByteBuffer;
@@ -458,22 +457,28 @@ abstract class DataStreamBaseTest extends BaseTest {
       Assert.assertEquals(reply.getType(), Type.STREAM_DATA);
     }
     try {
-      assertHeader(out.getHeader(), dataSize);
+      for (Server s : servers) {
+        assertHeader(s, out.getHeader(), dataSize);
+      }
     } catch (Throwable e) {
       throw new CompletionException(e);
     }
   }
 
-  void assertHeader(RaftClientRequest header, int dataSize) throws Exception {
-    final Server server = getPrimaryServer();
+  void assertHeader(Server server, RaftClientRequest header, int dataSize) throws Exception {
     final MultiDataStreamStateMachine s = server.getStateMachine(header.getRaftGroupId());
     final SingleDataStream stream = s.getSingleDataStream(header.getCallId());
     final RaftClientRequest writeRequest = stream.getWriteRequest();
     Assert.assertEquals(writeRequest.getCallId(), header.getCallId());
     Assert.assertEquals(writeRequest.getRaftGroupId(), header.getRaftGroupId());
     Assert.assertEquals(raftGroup.getGroupId(), header.getRaftGroupId());
-    Assert.assertEquals(writeRequest.getServerId(), header.getServerId());
     Assert.assertEquals(dataSize, stream.getByteWritten());
+    Assert.assertEquals(writeRequest.getCallId(), header.getCallId());
+
+    final Server primary = getPrimaryServer();
+    if (server == primary) {
+      Assert.assertEquals(writeRequest.getServerId(), header.getServerId());
+    }
   }
 
   static ByteBuffer initBuffer(int offset, int size) {
diff --git a/ratis-test/src/test/java/org/apache/ratis/datastream/TestDataStreamNetty.java b/ratis-test/src/test/java/org/apache/ratis/datastream/TestDataStreamNetty.java
index 7ae96b1..83decef 100644
--- a/ratis-test/src/test/java/org/apache/ratis/datastream/TestDataStreamNetty.java
+++ b/ratis-test/src/test/java/org/apache/ratis/datastream/TestDataStreamNetty.java
@@ -78,7 +78,7 @@ public class TestDataStreamNetty extends DataStreamBaseTest {
     RaftClientReply expectedClientReply = new RaftClientReply(clientId, suggestedLeader.getId(),
         groupId, callId, true, null, null, longIndex, null);
 
-    for (int i = 0; i < 3; i ++) {
+    for (int i = 0; i < numServers; i ++) {
       RaftServer raftServer = mock(RaftServer.class);
       RaftClientReply raftClientReply;
       RaftPeerId peerId = RaftPeerId.valueOf("s" + i);