You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@ratis.apache.org by sz...@apache.org on 2022/02/18 12:02:36 UTC

[ratis] branch master updated: RATIS-1513. Streaming tests may fail with TimeoutIOException: Timeout 3000ms (#604)

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

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


The following commit(s) were added to refs/heads/master by this push:
     new ffee9d0  RATIS-1513. Streaming tests may fail with TimeoutIOException: Timeout 3000ms (#604)
ffee9d0 is described below

commit ffee9d0f438030b3711b67ffee330720315ff226
Author: hao guo <gu...@360.cn>
AuthorDate: Fri Feb 18 20:02:18 2022 +0800

    RATIS-1513. Streaming tests may fail with TimeoutIOException: Timeout 3000ms (#604)
---
 .../java/org/apache/ratis/client/RaftClientConfigKeys.java     |  2 +-
 .../java/org/apache/ratis/client/impl/OrderedStreamAsync.java  | 10 +++++++---
 .../src/main/java/org/apache/ratis/io/WriteOption.java         |  4 ++++
 3 files changed, 12 insertions(+), 4 deletions(-)

diff --git a/ratis-client/src/main/java/org/apache/ratis/client/RaftClientConfigKeys.java b/ratis-client/src/main/java/org/apache/ratis/client/RaftClientConfigKeys.java
index c8388bb..0e950ee 100644
--- a/ratis-client/src/main/java/org/apache/ratis/client/RaftClientConfigKeys.java
+++ b/ratis-client/src/main/java/org/apache/ratis/client/RaftClientConfigKeys.java
@@ -105,7 +105,7 @@ public interface RaftClientConfigKeys {
     }
 
     String REQUEST_TIMEOUT_KEY = PREFIX + ".request.timeout";
-    TimeDuration REQUEST_TIMEOUT_DEFAULT = TimeDuration.valueOf(3000, TimeUnit.MILLISECONDS);
+    TimeDuration REQUEST_TIMEOUT_DEFAULT = TimeDuration.valueOf(10000, TimeUnit.MILLISECONDS);
     static TimeDuration requestTimeout(RaftProperties properties) {
       return getTimeDuration(properties.getTimeDuration(REQUEST_TIMEOUT_DEFAULT.getUnit()),
           REQUEST_TIMEOUT_KEY, REQUEST_TIMEOUT_DEFAULT, getDefaultLog());
diff --git a/ratis-client/src/main/java/org/apache/ratis/client/impl/OrderedStreamAsync.java b/ratis-client/src/main/java/org/apache/ratis/client/impl/OrderedStreamAsync.java
index d9f5862..5bc3242 100644
--- a/ratis-client/src/main/java/org/apache/ratis/client/impl/OrderedStreamAsync.java
+++ b/ratis-client/src/main/java/org/apache/ratis/client/impl/OrderedStreamAsync.java
@@ -24,6 +24,7 @@ import org.apache.ratis.datastream.impl.DataStreamPacketByteBuffer;
 import org.apache.ratis.datastream.impl.DataStreamRequestByteBuffer;
 import org.apache.ratis.datastream.impl.DataStreamRequestFilePositionCount;
 import org.apache.ratis.io.FilePositionCount;
+import org.apache.ratis.io.StandardWriteOption;
 import org.apache.ratis.protocol.DataStreamReply;
 import org.apache.ratis.protocol.DataStreamRequest;
 import org.apache.ratis.protocol.DataStreamRequestHeader;
@@ -105,12 +106,14 @@ public class OrderedStreamAsync {
 
   private final Semaphore requestSemaphore;
   private final TimeDuration requestTimeout;
+  private final TimeDuration closeTimeout;
   private final TimeoutScheduler scheduler = TimeoutScheduler.getInstance();
 
   OrderedStreamAsync(DataStreamClientRpc dataStreamClientRpc, RaftProperties properties){
     this.dataStreamClientRpc = dataStreamClientRpc;
     this.requestSemaphore = new Semaphore(RaftClientConfigKeys.DataStream.outstandingRequestsMax(properties));
     this.requestTimeout = RaftClientConfigKeys.DataStream.requestTimeout(properties);
+    this.closeTimeout = requestTimeout.multiply(2);
   }
 
   CompletableFuture<DataStreamReply> sendRequest(DataStreamRequestHeader header, Object data,
@@ -145,7 +148,8 @@ public class OrderedStreamAsync {
         request.getDataStreamRequest());
     long seqNum = request.getSeqNum();
 
-    scheduleWithTimeout(request);
+    final boolean isClose = StandardWriteOption.CLOSE.isOneOf(request.getDataStreamRequest().getWriteOptions());
+    scheduleWithTimeout(request, isClose? closeTimeout: requestTimeout);
 
     requestFuture.thenApply(reply -> {
       slidingWindow.receiveReply(
@@ -162,8 +166,8 @@ public class OrderedStreamAsync {
     });
   }
 
-  private void scheduleWithTimeout(DataStreamWindowRequest request) {
-    scheduler.onTimeout(requestTimeout, () -> {
+  private void scheduleWithTimeout(DataStreamWindowRequest request, TimeDuration timeout) {
+    scheduler.onTimeout(timeout, () -> {
       if (!request.getReplyFuture().isDone()) {
         request.getReplyFuture().completeExceptionally(
             new TimeoutIOException("Timeout " + requestTimeout + ": Failed to send " + request));
diff --git a/ratis-common/src/main/java/org/apache/ratis/io/WriteOption.java b/ratis-common/src/main/java/org/apache/ratis/io/WriteOption.java
index 11760c4..2768714 100644
--- a/ratis-common/src/main/java/org/apache/ratis/io/WriteOption.java
+++ b/ratis-common/src/main/java/org/apache/ratis/io/WriteOption.java
@@ -27,4 +27,8 @@ public interface WriteOption {
 
     return false;
   }
+
+  default boolean isOneOf(WriteOption... options) {
+    return containsOption(options, this);
+  }
 }