diff --git a/hadoop-hdds/common/src/main/java/org/apache/hadoop/hdds/scm/ScmConfigKeys.java b/hadoop-hdds/common/src/main/java/org/apache/hadoop/hdds/scm/ScmConfigKeys.java
index 062b1013c0c..38e6cbd7605 100644
--- a/hadoop-hdds/common/src/main/java/org/apache/hadoop/hdds/scm/ScmConfigKeys.java
+++ b/hadoop-hdds/common/src/main/java/org/apache/hadoop/hdds/scm/ScmConfigKeys.java
@@ -78,7 +78,7 @@ public final class ScmConfigKeys {
public static final String DFS_CONTAINER_RATIS_SEGMENT_PREALLOCATED_SIZE_KEY =
"dfs.container.ratis.segment.preallocated.size";
public static final String
- DFS_CONTAINER_RATIS_SEGMENT_PREALLOCATED_SIZE_DEFAULT = "128MB";
+ DFS_CONTAINER_RATIS_SEGMENT_PREALLOCATED_SIZE_DEFAULT = "16KB";
public static final String
DFS_CONTAINER_RATIS_STATEMACHINEDATA_SYNC_TIMEOUT =
"dfs.container.ratis.statemachinedata.sync.timeout";
@@ -90,10 +90,23 @@ public final class ScmConfigKeys {
"dfs.container.ratis.statemachinedata.sync.retries";
public static final int
DFS_CONTAINER_RATIS_STATEMACHINEDATA_SYNC_RETRIES_DEFAULT = -1;
- public static final String DFS_CONTAINER_RATIS_LOG_QUEUE_SIZE =
- "dfs.container.ratis.log.queue.size";
- public static final int DFS_CONTAINER_RATIS_LOG_QUEUE_SIZE_DEFAULT = 128;
-
+ public static final String DFS_CONTAINER_RATIS_LOG_QUEUE_NUM_ELEMENTS =
+ "dfs.container.ratis.log.queue.num-elements";
+ public static final int DFS_CONTAINER_RATIS_LOG_QUEUE_NUM_ELEMENTS_DEFAULT =
+ 1024;
+ public static final String DFS_CONTAINER_RATIS_LOG_QUEUE_BYTE_LIMIT =
+ "dfs.container.ratis.log.queue.byte-limit";
+ public static final String DFS_CONTAINER_RATIS_LOG_QUEUE_BYTE_LIMIT_DEFAULT =
+ "4GB";
+ public static final String
+ DFS_CONTAINER_RATIS_LOG_APPENDER_QUEUE_NUM_ELEMENTS =
+ "dfs.container.ratis.log.appender.queue.num-elements";
+ public static final int
+ DFS_CONTAINER_RATIS_LOG_APPENDER_QUEUE_NUM_ELEMENTS_DEFAULT = 1;
+ public static final String DFS_CONTAINER_RATIS_LOG_APPENDER_QUEUE_BYTE_LIMIT =
+ "dfs.container.ratis.log.appender.queue.byte-limit";
+ public static final String
+ DFS_CONTAINER_RATIS_LOG_APPENDER_QUEUE_BYTE_LIMIT_DEFAULT = "32MB";
// expiry interval stateMachineData cache entry inside containerStateMachine
public static final String
DFS_CONTAINER_RATIS_STATEMACHINEDATA_CACHE_EXPIRY_INTERVAL =
diff --git a/hadoop-hdds/common/src/main/java/org/apache/hadoop/ozone/OzoneConfigKeys.java b/hadoop-hdds/common/src/main/java/org/apache/hadoop/ozone/OzoneConfigKeys.java
index bac07d595c8..9c78daf313a 100644
--- a/hadoop-hdds/common/src/main/java/org/apache/hadoop/ozone/OzoneConfigKeys.java
+++ b/hadoop-hdds/common/src/main/java/org/apache/hadoop/ozone/OzoneConfigKeys.java
@@ -288,10 +288,25 @@ public final class OzoneConfigKeys {
public static final int
DFS_CONTAINER_RATIS_STATEMACHINEDATA_SYNC_RETRIES_DEFAULT =
ScmConfigKeys.DFS_CONTAINER_RATIS_STATEMACHINEDATA_SYNC_RETRIES_DEFAULT;
- public static final String DFS_CONTAINER_RATIS_LOG_QUEUE_SIZE =
- ScmConfigKeys.DFS_CONTAINER_RATIS_LOG_QUEUE_SIZE;
- public static final int DFS_CONTAINER_RATIS_LOG_QUEUE_SIZE_DEFAULT =
- ScmConfigKeys.DFS_CONTAINER_RATIS_LOG_QUEUE_SIZE_DEFAULT;
+ public static final String DFS_CONTAINER_RATIS_LOG_QUEUE_NUM_ELEMENTS =
+ ScmConfigKeys.DFS_CONTAINER_RATIS_LOG_QUEUE_NUM_ELEMENTS;
+ public static final int DFS_CONTAINER_RATIS_LOG_QUEUE_NUM_ELEMENTS_DEFAULT =
+ ScmConfigKeys.DFS_CONTAINER_RATIS_LOG_QUEUE_NUM_ELEMENTS_DEFAULT;
+ public static final String DFS_CONTAINER_RATIS_LOG_QUEUE_BYTE_LIMIT =
+ ScmConfigKeys.DFS_CONTAINER_RATIS_LOG_QUEUE_BYTE_LIMIT;
+ public static final String DFS_CONTAINER_RATIS_LOG_QUEUE_BYTE_LIMIT_DEFAULT =
+ ScmConfigKeys.DFS_CONTAINER_RATIS_LOG_QUEUE_BYTE_LIMIT_DEFAULT;
+ public static final String
+ DFS_CONTAINER_RATIS_LOG_APPENDER_QUEUE_NUM_ELEMENTS =
+ ScmConfigKeys.DFS_CONTAINER_RATIS_LOG_APPENDER_QUEUE_NUM_ELEMENTS;
+ public static final int
+ DFS_CONTAINER_RATIS_LOG_APPENDER_QUEUE_NUM_ELEMENTS_DEFAULT =
+ ScmConfigKeys.DFS_CONTAINER_RATIS_LOG_APPENDER_QUEUE_NUM_ELEMENTS_DEFAULT;
+ public static final String DFS_CONTAINER_RATIS_LOG_APPENDER_QUEUE_BYTE_LIMIT =
+ ScmConfigKeys.DFS_CONTAINER_RATIS_LOG_APPENDER_QUEUE_BYTE_LIMIT;
+ public static final String
+ DFS_CONTAINER_RATIS_LOG_APPENDER_QUEUE_BYTE_LIMIT_DEFAULT =
+ ScmConfigKeys.DFS_CONTAINER_RATIS_LOG_APPENDER_QUEUE_BYTE_LIMIT_DEFAULT;
public static final String DFS_RATIS_SERVER_REQUEST_TIMEOUT_DURATION_KEY =
ScmConfigKeys.DFS_RATIS_SERVER_REQUEST_TIMEOUT_DURATION_KEY;
public static final TimeDuration
diff --git a/hadoop-hdds/common/src/main/java/org/apache/ratis/RatisHelper.java b/hadoop-hdds/common/src/main/java/org/apache/ratis/RatisHelper.java
index e55dc5be840..9fbf9833e63 100644
--- a/hadoop-hdds/common/src/main/java/org/apache/ratis/RatisHelper.java
+++ b/hadoop-hdds/common/src/main/java/org/apache/ratis/RatisHelper.java
@@ -170,7 +170,7 @@ public interface RatisHelper {
long retryInterval = conf.getTimeDuration(OzoneConfigKeys.
DFS_RATIS_CLIENT_REQUEST_RETRY_INTERVAL_KEY, OzoneConfigKeys.
DFS_RATIS_CLIENT_REQUEST_RETRY_INTERVAL_DEFAULT
- .toInt(TimeUnit.MILLISECONDS), TimeUnit.MILLISECONDS);
+ .toIntExact(TimeUnit.MILLISECONDS), TimeUnit.MILLISECONDS);
TimeDuration sleepDuration =
TimeDuration.valueOf(retryInterval, TimeUnit.MILLISECONDS);
RetryPolicy retryPolicy = RetryPolicies
diff --git a/hadoop-hdds/common/src/main/resources/ozone-default.xml b/hadoop-hdds/common/src/main/resources/ozone-default.xml
index 2f4179a63ed..ad6745ba118 100644
--- a/hadoop-hdds/common/src/main/resources/ozone-default.xml
+++ b/hadoop-hdds/common/src/main/resources/ozone-default.xml
@@ -76,10 +76,32 @@
- dfs.container.ratis.log.queue.size
- 128
+ dfs.container.ratis.log.queue.num-elements
+ 1024
OZONE, DEBUG, CONTAINER, RATIS
- Number of operation pending with Raft's Log Worker.
+ Limit for the number of operations in Ratis Log Worker.
+
+
+
+ dfs.container.ratis.log.queue.byte-limit
+ 4GB
+ OZONE, DEBUG, CONTAINER, RATIS
+ Byte limit for Ratis Log Worker queue.
+
+
+
+ dfs.container.ratis.log.appender.queue.num-elements
+ 1
+ OZONE, DEBUG, CONTAINER, RATIS
+ Limit for number of append entries in ratis leader's
+ log appender queue.
+
+
+
+ dfs.container.ratis.log.appender.queue.byte-limit
+ 32MB
+ OZONE, DEBUG, CONTAINER, RATIS
+ Byte limit for ratis leader's log appender queue.
@@ -191,10 +213,10 @@
dfs.container.ratis.segment.preallocated.size
- 128MB
+ 16KB
OZONE, RATIS, PERFORMANCE
The size of the buffer which is preallocated for raft segment
- used by Apache Ratis on datanodes.(128 MB by default)
+ used by Apache Ratis on datanodes.(16 KB by default)
@@ -1476,10 +1498,25 @@
ozone.om.ratis.segment.preallocated.size
- 128MB
+ 16KB
OZONE, OM, RATIS, PERFORMANCE
The size of the buffer which is preallocated for raft segment
- used by Apache Ratis on OM.(128 MB by default)
+ used by Apache Ratis on OM.(16 KB by default)
+
+
+
+
+ ozone.om.ratis.log.appender.queue.num-elements
+ 1024
+ OZONE, DEBUG, CONTAINER, RATIS
+ Number of operation pending with Raft's Log Worker.
+
+
+
+ ozone.om.ratis.log.appender.queue.byte-limit
+ 32MB
+ OZONE, DEBUG, CONTAINER, RATIS
+ Byte limit for Raft's Log Worker queue.
diff --git a/hadoop-hdds/container-service/src/main/java/org/apache/hadoop/ozone/container/common/transport/server/ratis/ContainerStateMachine.java b/hadoop-hdds/container-service/src/main/java/org/apache/hadoop/ozone/container/common/transport/server/ratis/ContainerStateMachine.java
index b693e9e8cbc..68ee91c6c72 100644
--- a/hadoop-hdds/container-service/src/main/java/org/apache/hadoop/ozone/container/common/transport/server/ratis/ContainerStateMachine.java
+++ b/hadoop-hdds/container-service/src/main/java/org/apache/hadoop/ozone/container/common/transport/server/ratis/ContainerStateMachine.java
@@ -22,6 +22,7 @@ import com.google.common.base.Preconditions;
import com.google.common.cache.Cache;
import com.google.common.cache.CacheBuilder;
import org.apache.hadoop.hdds.HddsUtils;
+import org.apache.hadoop.hdds.protocol.datanode.proto.ContainerProtos;
import org.apache.ratis.proto.RaftProtos.RaftPeerRole;
import org.apache.ratis.protocol.RaftGroup;
import org.apache.ratis.protocol.RaftGroupId;
@@ -388,11 +389,12 @@ public class ContainerStateMachine extends BaseStateMachine {
ContainerCommandRequestProto requestProto, long term, long index) {
WriteChunkRequestProto writeChunkRequestProto =
requestProto.getWriteChunk();
+ ContainerProtos.ChunkInfo chunkInfo = writeChunkRequestProto.getChunkData();
// prepare the chunk to be read
ReadChunkRequestProto.Builder readChunkRequestProto =
ReadChunkRequestProto.newBuilder()
.setBlockID(writeChunkRequestProto.getBlockID())
- .setChunkData(writeChunkRequestProto.getChunkData());
+ .setChunkData(chunkInfo);
ContainerCommandRequestProto dataContainerCommandProto =
ContainerCommandRequestProto.newBuilder(requestProto)
.setCmdType(Type.ReadChunk)
@@ -411,7 +413,11 @@ public class ContainerStateMachine extends BaseStateMachine {
ByteString data = responseProto.getData();
// assert that the response has data in it.
- Preconditions.checkNotNull(data);
+ Preconditions
+ .checkNotNull(data, "read chunk data is null for chunk:" + chunkInfo);
+ Preconditions.checkState(data.size() == chunkInfo.getLen(), String.format(
+ "read chunk len=%d does not match chunk expected len=%d for chunk:%s",
+ data.size(), chunkInfo.getLen(), chunkInfo));
return data;
}
diff --git a/hadoop-hdds/container-service/src/main/java/org/apache/hadoop/ozone/container/common/transport/server/ratis/XceiverServerRatis.java b/hadoop-hdds/container-service/src/main/java/org/apache/hadoop/ozone/container/common/transport/server/ratis/XceiverServerRatis.java
index 2675483a9e9..5abc18541c2 100644
--- a/hadoop-hdds/container-service/src/main/java/org/apache/hadoop/ozone/container/common/transport/server/ratis/XceiverServerRatis.java
+++ b/hadoop-hdds/container-service/src/main/java/org/apache/hadoop/ozone/container/common/transport/server/ratis/XceiverServerRatis.java
@@ -175,8 +175,19 @@ public final class XceiverServerRatis implements XceiverServerSpi {
OzoneConfigKeys.DFS_CONTAINER_RATIS_SEGMENT_PREALLOCATED_SIZE_KEY,
OzoneConfigKeys.DFS_CONTAINER_RATIS_SEGMENT_PREALLOCATED_SIZE_DEFAULT,
StorageUnit.BYTES);
- RaftServerConfigKeys.Log.Appender.setBufferCapacity(properties,
- SizeInBytes.valueOf(raftSegmentPreallocatedSize));
+ int logAppenderQueueNumElements = conf.getInt(
+ OzoneConfigKeys.DFS_CONTAINER_RATIS_LOG_APPENDER_QUEUE_NUM_ELEMENTS,
+ OzoneConfigKeys
+ .DFS_CONTAINER_RATIS_LOG_APPENDER_QUEUE_NUM_ELEMENTS_DEFAULT);
+ final int logAppenderQueueByteLimit = (int) conf.getStorageSize(
+ OzoneConfigKeys.DFS_CONTAINER_RATIS_LOG_APPENDER_QUEUE_BYTE_LIMIT,
+ OzoneConfigKeys
+ .DFS_CONTAINER_RATIS_LOG_APPENDER_QUEUE_BYTE_LIMIT_DEFAULT,
+ StorageUnit.BYTES);
+ RaftServerConfigKeys.Log.Appender
+ .setBufferElementLimit(properties, logAppenderQueueNumElements);
+ RaftServerConfigKeys.Log.Appender.setBufferByteLimit(properties,
+ SizeInBytes.valueOf(logAppenderQueueByteLimit));
RaftServerConfigKeys.Log.setPreallocatedSize(properties,
SizeInBytes.valueOf(raftSegmentPreallocatedSize));
@@ -255,8 +266,6 @@ public final class XceiverServerRatis implements XceiverServerSpi {
leaderElectionMinTimeout.toLong(TimeUnit.MILLISECONDS) + 200;
RaftServerConfigKeys.Rpc.setTimeoutMax(properties,
TimeDuration.valueOf(leaderElectionMaxTimeout, TimeUnit.MILLISECONDS));
- // Enable batch append on raft server
- RaftServerConfigKeys.Log.Appender.setBatchEnabled(properties, true);
// Set the maximum cache segments
RaftServerConfigKeys.Log.setMaxCachedSegmentNum(properties, 2);
@@ -299,10 +308,15 @@ public final class XceiverServerRatis implements XceiverServerSpi {
setAutoTriggerEnabled(properties, true);
RaftServerConfigKeys.Snapshot.
setAutoTriggerThreshold(properties, snapshotThreshold);
- int logQueueSize =
- conf.getInt(OzoneConfigKeys.DFS_CONTAINER_RATIS_LOG_QUEUE_SIZE,
- OzoneConfigKeys.DFS_CONTAINER_RATIS_LOG_QUEUE_SIZE_DEFAULT);
- RaftServerConfigKeys.Log.setQueueSize(properties, logQueueSize);
+ int logQueueNumElements =
+ conf.getInt(OzoneConfigKeys.DFS_CONTAINER_RATIS_LOG_QUEUE_NUM_ELEMENTS,
+ OzoneConfigKeys.DFS_CONTAINER_RATIS_LOG_QUEUE_NUM_ELEMENTS_DEFAULT);
+ final int logQueueByteLimit = (int) conf.getStorageSize(
+ OzoneConfigKeys.DFS_CONTAINER_RATIS_LOG_QUEUE_BYTE_LIMIT,
+ OzoneConfigKeys.DFS_CONTAINER_RATIS_LOG_QUEUE_BYTE_LIMIT_DEFAULT,
+ StorageUnit.BYTES);
+ RaftServerConfigKeys.Log.setElementLimit(properties, logQueueNumElements);
+ RaftServerConfigKeys.Log.setByteLimit(properties, logQueueByteLimit);
int numSyncRetries = conf.getInt(
OzoneConfigKeys.DFS_CONTAINER_RATIS_STATEMACHINEDATA_SYNC_RETRIES,
@@ -409,7 +423,7 @@ public final class XceiverServerRatis implements XceiverServerSpi {
RaftClientReply reply;
RaftClientRequest raftClientRequest =
createRaftClientRequest(request, pipelineID,
- RaftClientRequest.writeRequestType(replicationLevel));
+ RaftClientRequest.writeRequestType());
try {
reply = server.submitClientRequestAsync(raftClientRequest).get();
} catch (Exception e) {
diff --git a/hadoop-hdds/pom.xml b/hadoop-hdds/pom.xml
index 5537b3a2265..f395f85d68e 100644
--- a/hadoop-hdds/pom.xml
+++ b/hadoop-hdds/pom.xml
@@ -46,7 +46,7 @@ http://maven.apache.org/xsd/maven-4.0.0.xsd">
0.4.0-SNAPSHOT
- 0.4.0-b600fc2-SNAPSHOT
+ 0.4.0-3b0be02-SNAPSHOT
1.60
diff --git a/hadoop-hdds/server-scm/src/main/java/org/apache/hadoop/hdds/scm/pipeline/RatisPipelineUtils.java b/hadoop-hdds/server-scm/src/main/java/org/apache/hadoop/hdds/scm/pipeline/RatisPipelineUtils.java
index 6cf3abe30c6..e58b3ff9aa9 100644
--- a/hadoop-hdds/server-scm/src/main/java/org/apache/hadoop/hdds/scm/pipeline/RatisPipelineUtils.java
+++ b/hadoop-hdds/server-scm/src/main/java/org/apache/hadoop/hdds/scm/pipeline/RatisPipelineUtils.java
@@ -28,7 +28,7 @@ import org.apache.ratis.protocol.RaftGroupId;
import org.apache.ratis.protocol.RaftPeer;
import org.apache.ratis.retry.RetryPolicy;
import org.apache.ratis.rpc.SupportedRpcType;
-import org.apache.ratis.util.CheckedBiConsumer;
+import org.apache.ratis.util.function.CheckedBiConsumer;
import org.apache.ratis.util.TimeDuration;
import org.apache.ratis.util.TimeoutScheduler;
import org.slf4j.Logger;
diff --git a/hadoop-ozone/common/src/main/java/org/apache/hadoop/ozone/om/OMConfigKeys.java b/hadoop-ozone/common/src/main/java/org/apache/hadoop/ozone/om/OMConfigKeys.java
index 423d2516715..82eca4e8380 100644
--- a/hadoop-ozone/common/src/main/java/org/apache/hadoop/ozone/om/OMConfigKeys.java
+++ b/hadoop-ozone/common/src/main/java/org/apache/hadoop/ozone/om/OMConfigKeys.java
@@ -121,7 +121,19 @@ public final class OMConfigKeys {
public static final String OZONE_OM_RATIS_SEGMENT_PREALLOCATED_SIZE_KEY
= "ozone.om.ratis.segment.preallocated.size";
public static final String OZONE_OM_RATIS_SEGMENT_PREALLOCATED_SIZE_DEFAULT
- = "128MB";
+ = "16KB";
+
+ // OM Ratis Log Appender configurations
+ public static final String
+ OZONE_OM_RATIS_LOG_APPENDER_QUEUE_NUM_ELEMENTS =
+ "ozone.om.ratis.log.appender.queue.num-elements";
+ public static final int
+ OZONE_OM_RATIS_LOG_APPENDER_QUEUE_NUM_ELEMENTS_DEFAULT = 1024;
+ public static final String OZONE_OM_RATIS_LOG_APPENDER_QUEUE_BYTE_LIMIT =
+ "ozone.om.ratis.log.appender.queue.byte-limit";
+ public static final String
+ OZONE_OM_RATIS_LOG_APPENDER_QUEUE_BYTE_LIMIT_DEFAULT = "32MB";
+
// OM Ratis server configurations
public static final String OZONE_OM_RATIS_SERVER_REQUEST_TIMEOUT_KEY
diff --git a/hadoop-ozone/integration-test/src/test/java/org/apache/hadoop/ozone/OzoneTestUtils.java b/hadoop-ozone/integration-test/src/test/java/org/apache/hadoop/ozone/OzoneTestUtils.java
index 81cb80574ab..c4e6da165f5 100644
--- a/hadoop-ozone/integration-test/src/test/java/org/apache/hadoop/ozone/OzoneTestUtils.java
+++ b/hadoop-ozone/integration-test/src/test/java/org/apache/hadoop/ozone/OzoneTestUtils.java
@@ -23,7 +23,7 @@ import org.apache.hadoop.hdds.scm.container.ContainerID;
import org.apache.hadoop.hdds.scm.server.StorageContainerManager;
import org.apache.hadoop.ozone.om.helpers.OmKeyLocationInfo;
import org.apache.hadoop.ozone.om.helpers.OmKeyLocationInfoGroup;
-import org.apache.ratis.util.CheckedConsumer;
+import org.apache.ratis.util.function.CheckedConsumer;
import org.junit.Assert;
import java.io.IOException;
diff --git a/hadoop-ozone/integration-test/src/test/java/org/apache/hadoop/ozone/container/common/transport/server/ratis/TestCSMMetrics.java b/hadoop-ozone/integration-test/src/test/java/org/apache/hadoop/ozone/container/common/transport/server/ratis/TestCSMMetrics.java
index 43606f0bddf..d0a991c446f 100644
--- a/hadoop-ozone/integration-test/src/test/java/org/apache/hadoop/ozone/container/common/transport/server/ratis/TestCSMMetrics.java
+++ b/hadoop-ozone/integration-test/src/test/java/org/apache/hadoop/ozone/container/common/transport/server/ratis/TestCSMMetrics.java
@@ -49,7 +49,7 @@ import org.apache.hadoop.hdds.conf.OzoneConfiguration;
import static org.apache.ratis.rpc.SupportedRpcType.GRPC;
import org.apache.ratis.protocol.RaftGroupId;
-import org.apache.ratis.util.CheckedBiConsumer;
+import org.apache.ratis.util.function.CheckedBiConsumer;
import java.util.function.BiConsumer;
diff --git a/hadoop-ozone/integration-test/src/test/java/org/apache/hadoop/ozone/container/ozoneimpl/TestOzoneContainerRatis.java b/hadoop-ozone/integration-test/src/test/java/org/apache/hadoop/ozone/container/ozoneimpl/TestOzoneContainerRatis.java
index 13e41e26c49..8577156a40e 100644
--- a/hadoop-ozone/integration-test/src/test/java/org/apache/hadoop/ozone/container/ozoneimpl/TestOzoneContainerRatis.java
+++ b/hadoop-ozone/integration-test/src/test/java/org/apache/hadoop/ozone/container/ozoneimpl/TestOzoneContainerRatis.java
@@ -28,7 +28,7 @@ import org.apache.hadoop.hdds.scm.XceiverClientSpi;
import org.apache.hadoop.hdds.scm.pipeline.Pipeline;
import org.apache.ratis.rpc.RpcType;
import org.apache.ratis.rpc.SupportedRpcType;
-import org.apache.ratis.util.CheckedBiConsumer;
+import org.apache.ratis.util.function.CheckedBiConsumer;
import org.apache.ratis.util.CollectionUtils;
import org.junit.Ignore;
import org.junit.Rule;
diff --git a/hadoop-ozone/integration-test/src/test/java/org/apache/hadoop/ozone/container/server/TestContainerServer.java b/hadoop-ozone/integration-test/src/test/java/org/apache/hadoop/ozone/container/server/TestContainerServer.java
index 16754846125..d36c2535847 100644
--- a/hadoop-ozone/integration-test/src/test/java/org/apache/hadoop/ozone/container/server/TestContainerServer.java
+++ b/hadoop-ozone/integration-test/src/test/java/org/apache/hadoop/ozone/container/server/TestContainerServer.java
@@ -52,7 +52,7 @@ import org.apache.hadoop.hdds.scm.XceiverClientSpi;
import org.apache.hadoop.hdds.scm.pipeline.Pipeline;
import org.apache.hadoop.test.GenericTestUtils;
import org.apache.ratis.rpc.RpcType;
-import org.apache.ratis.util.CheckedBiConsumer;
+import org.apache.ratis.util.function.CheckedBiConsumer;
import org.junit.Assert;
import org.junit.Ignore;
import org.junit.Test;
diff --git a/hadoop-ozone/ozone-manager/src/main/java/org/apache/hadoop/ozone/om/ratis/OzoneManagerRatisServer.java b/hadoop-ozone/ozone-manager/src/main/java/org/apache/hadoop/ozone/om/ratis/OzoneManagerRatisServer.java
index de1fd6070dc..2b410072792 100644
--- a/hadoop-ozone/ozone-manager/src/main/java/org/apache/hadoop/ozone/om/ratis/OzoneManagerRatisServer.java
+++ b/hadoop-ozone/ozone-manager/src/main/java/org/apache/hadoop/ozone/om/ratis/OzoneManagerRatisServer.java
@@ -156,8 +156,17 @@ public final class OzoneManagerRatisServer {
OMConfigKeys.OZONE_OM_RATIS_SEGMENT_PREALLOCATED_SIZE_KEY,
OMConfigKeys.OZONE_OM_RATIS_SEGMENT_PREALLOCATED_SIZE_DEFAULT,
StorageUnit.BYTES);
- RaftServerConfigKeys.Log.Appender.setBufferCapacity(properties,
- SizeInBytes.valueOf(raftSegmentPreallocatedSize));
+ int logAppenderQueueNumElements = conf.getInt(
+ OMConfigKeys.OZONE_OM_RATIS_LOG_APPENDER_QUEUE_NUM_ELEMENTS,
+ OMConfigKeys.OZONE_OM_RATIS_LOG_APPENDER_QUEUE_NUM_ELEMENTS_DEFAULT);
+ final int logAppenderQueueByteLimit = (int) conf.getStorageSize(
+ OMConfigKeys.OZONE_OM_RATIS_LOG_APPENDER_QUEUE_BYTE_LIMIT,
+ OMConfigKeys.OZONE_OM_RATIS_LOG_APPENDER_QUEUE_BYTE_LIMIT_DEFAULT,
+ StorageUnit.BYTES);
+ RaftServerConfigKeys.Log.Appender
+ .setBufferElementLimit(properties, logAppenderQueueNumElements);
+ RaftServerConfigKeys.Log.Appender.setBufferByteLimit(properties,
+ SizeInBytes.valueOf(logAppenderQueueByteLimit));
RaftServerConfigKeys.Log.setPreallocatedSize(properties,
SizeInBytes.valueOf(raftSegmentPreallocatedSize));
@@ -165,7 +174,7 @@ public final class OzoneManagerRatisServer {
// TODO: calculate the max message size based on the max size of a
// PutSmallFileRequest's file size limit
GrpcConfigKeys.setMessageSizeMax(properties,
- SizeInBytes.valueOf(raftSegmentPreallocatedSize));
+ SizeInBytes.valueOf(logAppenderQueueByteLimit));
// Set the server request timeout
TimeUnit serverRequestTimeoutUnit =
@@ -209,9 +218,6 @@ public final class OzoneManagerRatisServer {
RaftServerConfigKeys.Rpc.setTimeoutMax(properties,
serverMaxTimeout);
- // Enable batch append on raft server
- RaftServerConfigKeys.Log.Appender.setBatchEnabled(properties, true);
-
// Set the number of maximum cached segments
RaftServerConfigKeys.Log.setMaxCachedSegmentNum(properties, 2);
diff --git a/hadoop-ozone/pom.xml b/hadoop-ozone/pom.xml
index 4c13bd6a3c3..486c2c443ea 100644
--- a/hadoop-ozone/pom.xml
+++ b/hadoop-ozone/pom.xml
@@ -33,7 +33,7 @@ http://maven.apache.org/xsd/maven-4.0.0.xsd">
3.2.1-SNAPSHOT
0.4.0-SNAPSHOT
0.4.0-SNAPSHOT
- 0.4.0-b600fc2-SNAPSHOT
+ 0.4.0-3b0be02-SNAPSHOT
1.60
Badlands
${ozone.version}