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 86300a1  RATIS-1089. Add getDataStreamApi() to RaftClient. (#215)
86300a1 is described below

commit 86300a172a6b3e90ead5117122abfd3e3221894d
Author: Tsz-Wo Nicholas Sze <[email protected]>
AuthorDate: Mon Oct 12 16:40:00 2020 +0800

    RATIS-1089. Add getDataStreamApi() to RaftClient. (#215)
    
    * RATIS-1089. Add getDataStreamApi() to RaftClient.
    
    * Revised the javadoc in DataStreamApi.
    
    * Remove unused import.
---
 .../org/apache/ratis/client/DataStreamClient.java  |  7 ++--
 .../java/org/apache/ratis/client/RaftClient.java   | 11 +++++-
 .../org/apache/ratis/client/api/DataStreamApi.java | 40 ++++++++++++++++++++++
 .../apache/ratis/client/api/MessageStreamApi.java  | 17 ++++++++-
 4 files changed, 68 insertions(+), 7 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 5762ca0..f49dc5f 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,7 @@
  */
 package org.apache.ratis.client;
 
-import org.apache.ratis.client.api.DataStreamOutput;
+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;
@@ -29,16 +29,13 @@ import org.slf4j.LoggerFactory;
  * A client interface that sends request to the streaming pipeline.
  * Associated with it will be a Netty Client.
  */
-public interface DataStreamClient {
+public interface DataStreamClient extends DataStreamApi {
 
   Logger LOG = LoggerFactory.getLogger(DataStreamClient.class);
 
   /** Return the rpc client instance **/
   DataStreamClientRpc getClientRpc();
 
-  /** @return a new output stream. */
-  DataStreamOutput stream();
-
   /** add information of the raft peers to communicate with */
   void addPeers(Iterable<RaftPeer> peers);
 
diff --git a/ratis-client/src/main/java/org/apache/ratis/client/RaftClient.java 
b/ratis-client/src/main/java/org/apache/ratis/client/RaftClient.java
index dc2b2af..41e3451 100644
--- a/ratis-client/src/main/java/org/apache/ratis/client/RaftClient.java
+++ b/ratis-client/src/main/java/org/apache/ratis/client/RaftClient.java
@@ -18,6 +18,7 @@
 package org.apache.ratis.client;
 
 import org.apache.ratis.RaftConfigKeys;
+import org.apache.ratis.client.api.DataStreamApi;
 import org.apache.ratis.client.api.MessageStreamApi;
 import org.apache.ratis.client.impl.ClientImplUtils;
 import org.apache.ratis.conf.Parameters;
@@ -27,6 +28,7 @@ import org.apache.ratis.retry.RetryPolicies;
 import org.apache.ratis.retry.RetryPolicy;
 import org.apache.ratis.rpc.RpcType;
 import org.apache.ratis.proto.RaftProtos.ReplicationLevel;
+import org.apache.ratis.util.JavaUtils;
 import org.slf4j.Logger;
 import org.slf4j.LoggerFactory;
 
@@ -45,12 +47,19 @@ public interface RaftClient extends Closeable {
   /** @return the cluster leaderId recorded by this client. */
   RaftPeerId getLeaderId();
 
-  /** @return the client rpct. */
+  /** @return the {@link RaftClientRpc}. */
   RaftClientRpc getClientRpc();
 
   /** @return the {@link MessageStreamApi}. */
   MessageStreamApi getMessageStreamApi();
 
+  /** @return the {@link DataStreamApi}. */
+  default DataStreamApi getDataStreamApi() {
+    // TODO RATIS-1090: Implements this once the streaming feature has become 
usable.
+    throw new UnsupportedOperationException(
+        JavaUtils.getCurrentStackTraceElement().getMethodName() + " is not yet 
supported.");
+  }
+
   /**
    * Async call to send the given message to the raft service.
    * The message may change the state of the service.
diff --git 
a/ratis-client/src/main/java/org/apache/ratis/client/api/DataStreamApi.java 
b/ratis-client/src/main/java/org/apache/ratis/client/api/DataStreamApi.java
new file mode 100644
index 0000000..467dfb5
--- /dev/null
+++ b/ratis-client/src/main/java/org/apache/ratis/client/api/DataStreamApi.java
@@ -0,0 +1,40 @@
+/*
+ * 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.api;
+
+/**
+ * Stream data asynchronously to all the servers in the {@link 
org.apache.ratis.protocol.RaftGroup}.
+ * Clients may stream data to the nearest server and then the server will 
forward the data to the other servers.
+ * Once all the servers have received all the data of a request,
+ * the leader (may or may not be the nearest server) creates a log entry with 
a data ID generated by the state machine.
+ * Then, the leader sends the log entry to the followers.
+ * Since the followers already have received the data from the stream,
+ * they may lookup the data from the ID.
+ *
+ * Since this API allows clients to send data to the nearest server which is 
not necessarily the leader,
+ * this API is more efficient for network-topology-aware clusters
+ * than the other APIs that require clients to send data/messages to the 
leader.
+ *
+ * Note that this API is different from {@link MessageStreamApi} in the sense 
that
+ * this API streams data to all the servers in the {@link 
org.apache.ratis.protocol.RaftGroup}
+ * but {@link MessageStreamApi} streams messages only to the leader.
+ */
+public interface DataStreamApi {
+  /** Create a stream to send data. */
+  DataStreamOutput stream();
+}
diff --git 
a/ratis-client/src/main/java/org/apache/ratis/client/api/MessageStreamApi.java 
b/ratis-client/src/main/java/org/apache/ratis/client/api/MessageStreamApi.java
index 4d0f9a7..27cfa7f 100644
--- 
a/ratis-client/src/main/java/org/apache/ratis/client/api/MessageStreamApi.java
+++ 
b/ratis-client/src/main/java/org/apache/ratis/client/api/MessageStreamApi.java
@@ -25,7 +25,22 @@ import org.slf4j.LoggerFactory;
 
 import java.util.concurrent.CompletableFuture;
 
-/** A client who sends requests to a raft service. */
+/**
+ * Stream (large) {@link Message}(s) asynchronously to the leader.
+ * Once a stream has been closed or has been signaled an end-of-request,
+ * the leader creates a raft log entry for the request
+ * and then replicates the log entry to all the followers.
+ *
+ * Note that this API is similar to {@link 
org.apache.ratis.client.RaftClient#sendAsync(Message)}
+ * except that {@link MessageStreamApi} divides a (large) message into 
multiple (small) sub-messages in the stream
+ * but {@link org.apache.ratis.client.RaftClient#sendAsync(Message)} sends the 
entire message in a single RPC.
+ * For sending large messages,
+ * {@link MessageStreamApi} is more efficient than {@link 
org.apache.ratis.client.RaftClient#sendAsync(Message)}}.
+ *
+ * Note also that this API is different from {@link DataStreamApi} in the 
sense that
+ * this API streams messages only to the leader
+ * but {@link DataStreamApi} streams data to all the servers in the {@link 
org.apache.ratis.protocol.RaftGroup}.
+ */
 public interface MessageStreamApi {
   Logger LOG = LoggerFactory.getLogger(MessageStreamApi.class);
 

Reply via email to