You are viewing a plain text version of this content. The canonical link for it is here.
Posted to issues@ratis.apache.org by GitBox <gi...@apache.org> on 2022/10/27 14:53:46 UTC

[GitHub] [ratis] codings-dan commented on a diff in pull request #770: RATIS-1732. Add configuration reference for RaftServer

codings-dan commented on code in PR #770:
URL: https://github.com/apache/ratis/pull/770#discussion_r1006841760


##########
ratis-docs/src/site/markdown/configuraions.md:
##########
@@ -0,0 +1,563 @@
+<!---
+  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.
+-->
+
+# Ratis Configuration Reference
+
+## Server Configurations
+
+Most of the server configurations can be found at `RaftServerConfigKeys`.
+To customize configurations, we may 
+1. create a `RaftProperties` object,
+2. set the desired values, and then
+3. pass the customized `RaftProperties` object when building a `RaftServer`.
+
+For example,
+```java
+final RaftProperties properties = new RaftProperties();
+RaftServerConfigKeys.LeaderElection.setPreVote(properties, false);
+final RaftServer raftServer = RaftServer.newBuilder()
+    .setServerId(id)
+    .setStateMachine(stateMachine)
+    .setGroup(raftGroup)
+    .setProperties(properties)
+    .build();
+```
+
+### Server
+
+| **Property**    | `raft.server.storage.dir`                      |
+|-----------------|------------------------------------------------|
+| **Description** | root storage directory to hold RaftServer data |
+| **Type**        | List\<File\>                                   |
+| **Default**     | /tmp/raft-server/                              |
+
+| **Property**    | `raft.server.storage.free-space.min`      |
+|-----------------|-------------------------------------------|
+| **Description** | minimal space requirement for storage dir |
+| **Type**        | SizeInBytes                               |
+| **Default**     | 0MB                                       |
+
+| **Property**    | `raft.server.removed.groups.dir`         |
+|-----------------|------------------------------------------|
+| **Description** | storage directory to hold removed groups |
+| **Type**        | string                                   |
+| **Default**     | /tmp/raft-server/removed-groups/         |
+
+```java
+// GroupManagementApi
+RaftClientReply remove(RaftGroupId groupId,
+    boolean deleteDirectory, boolean renameDirectory) throws IOException;
+```
+When removing an existing group,
+if the `deleteDirectory` flag is set to false and `renameDirectory` is set to true,
+the group data will be renamed to this dir instead of being deleted.
+
+| **Property**    | `raft.server.sleep.deviation.threshold` |
+|:----------------|-----------------------------------------|
+| **Description** | deviation threshold for election sleep  |
+| **Type**        | TimeDuration                            |
+| **Default**     | 300ms                                   |
+
+When a server is a follower,
+it sleeps and wakes up from time to time 
+for checking the heartbeat condition.
+If it cannot receive a heartbeat from the leader
+within the election timeout,
+it starts a leader election.
+
+When a follower server wakes up from a sleep,
+if the actual sleep time is longer
+than the intended sleep time by this threshold,
+it will immediately go back to sleep again,
+instead of checking the heartbeat condition.
+The extra sleep time indicates that
+the server is too busy,
+probably due to GC.
+
+| **Property**    | `raft.server.staging.catchup.gap`  |
+|-----------------|------------------------------------|
+| **Description** | catching up standard of a new peer |
+| **Type**        | int                                |
+| **Default**     | 1000                               |
+
+When bootstrapping a new peer, If the gap between the match index of the
+peer and the leader's latest committed index is less than this gap, we
+treat the peer as caught-up. Increase this number when write throughput is high.
+
+### ThreadPool
+
+Configurations related to RaftServer thread pools.
+
+* proxy thread pool: threads that recover and initialize RaftGroups when RaftServer starts.
+* server thread pool: threads that handle internal RPCs, like appendEntries.
+* client thread pool: threads that handle client requests, like client.io().send().
+
+| **Property**    | `raft.server.threadpool.proxy.cached` |
+|-----------------|---------------------------------------|
+| **Description** | use CachedThreadPool for proxy-pool   |
+| **Type**        | boolean                               |
+| **Default**     | true                                  |
+
+| **Property**    | `raft.server.threadpool.proxy.size` |
+|-----------------|-------------------------------------|
+| **Description** | maximumPoolSize for proxy-pool      |
+| **Type**        | int                                 |
+| **Default**     | 0                                   |
+
+| **Property**    | `raft.server.threadpool.server.cached` |
+|-----------------|----------------------------------------|
+| **Description** | use CachedThreadPool for server-pool   |
+| **Type**        | boolean                                |
+| **Default**     | true                                   |
+
+| **Property**    | `raft.server.threadpool.proxy.size` |
+|-----------------|-------------------------------------|
+| **Description** | maximumPoolSize for server-pool     |
+| **Type**        | int                                 |
+| **Default**     | 0                                   |
+
+| **Property**    | `raft.server.threadpool.client.cached` |
+|-----------------|----------------------------------------|
+| **Description** | use CachedThreadPool for client-pool   |
+| **Type**        | boolean                                |
+| **Default**     | true                                   |
+
+| **Property**    | `raft.server.threadpool.client.size` |
+|-----------------|--------------------------------------|
+| **Description** | maximumPoolSize for client-pool      |
+| **Type**        | int                                  |
+| **Default**     | 0                                    |
+
+### Read
+
+Configurations related to read-only requests.
+
+| **Property**    | `raft.server.read.option` |
+|-----------------|---------------------------|
+| **Description** | read-only option          |
+| **Type**        | enum                      |
+| **Default**     | Option.DEFAULT            |
+
+1. `Option.DEFAULT`: Directly query statemachine. Efficient but may undermine linearizability.
+2. `Option.LINEARIZABLE`: Use ReadIndex (see Raft Paper section 6.4). Maintains linearizability.
+
+| **Property**    | `raft.server.read.timeout`                          |
+|-----------------|-----------------------------------------------------|
+| **Description** | request timeout for linearizable read-only requests |
+| **Type**        | TimeDuration                                        |
+| **Default**     | 10s                                                 |
+
+### Write
+
+Configurations related to write requests.
+
+| **Property**    | `raft.server.write.element-limit` |
+|-----------------|-----------------------------------|
+| **Description** | max pending write requests        |
+| **Type**        | int                               |
+| **Default**     | 4096                              |
+
+| **Property**    | `raft.server.write.byte-limit`      |
+|-----------------|-------------------------------------|
+| **Description** | max pending write requests sum size |
+| **Type**        | SizeInBytes                         |
+| **Default**     | 64MB                                |
+
+Ratis imposes limitations on pending write requests. If the pending requests queue size exceeds element-limit or the
+size of bytes accumulated in pending requests exceeds byte-limit, RaftServer will reject incoming write requests until
+the pending situation is relieved.
+
+| **Property**    | `raft.server.write.follower.gap.ratio.max`                                                                              |
+|-----------------|-------------------------------------------------------------------------------------------------------------------------|
+| **Description** | set a threshold between the majority committed index and slow follower's committed index to guarantee the data in cache |
+| **Type**        | int                                                                                                                     |
+| **Default**     | -1, disable the feature                                                                                                 |
+
+### Watch
+
+Configurations related to watch requests.
+
+| **Property**    | `raft.server.watch.element-limit` |
+|-----------------|-----------------------------------|
+| **Description** | max pending watch requests        |
+| **Type**        | int                               |
+| **Default**     | 65536                             |
+
+| **Property**    | `raft.server.watch.timeout` |
+|-----------------|-----------------------------|
+| **Description** | watch request timeout       |
+| **Type**        | TimeDuration                |
+| **Default**     | 10s                         |
+
+| **Property**    | `raft.server.watch.timeout.denomination`           |
+|-----------------|----------------------------------------------------|
+| **Description** | watch request timeout denomination for rounding up |
+| **Type**        | TimeDuration                                       |
+| **Default**     | 1s                                                 |
+
+watch.timeout should be a multiple of watch.timeout.denomination
+
+### Log
+
+Configurations related to raft log.
+
+| **Property**    | `raft.server.log.use.memory` |
+|-----------------|------------------------------|
+| **Description** | use memory RaftLog           |
+| **Type**        | boolean                      |
+| **Default**     | false                        |
+
+| **Property**    | `raft.server.log.queue.element-limit` |
+|-----------------|---------------------------------------|
+| **Description** | max pending log tasks                 |
+| **Type**        | int                                   |
+| **Default**     | 4096                                  |
+
+| **Property**    | `raft.server.log.queue.byte-limit`             |
+|-----------------|------------------------------------------------|
+| **Description** | size of bytes accumulated in pending log tasks |
+| **Type**        | SizeInBytes                                    |
+| **Default**     | 64MB                                           |
+
+log.queue.byte-limit and log.queue.element-limit are quite similar to write.element-limit and .write.byte-limit. When
+pending IO tasks reached the limit, Ratis will temporarily stall new IO Tasks.
+
+| **Property**    | `raft.server.log.purge.gap`             |
+|-----------------|-----------------------------------------|
+| **Description** | minimal log gap between two purge tasks |
+| **Type**        | int                                     |
+| **Default**     | 1024                                    |
+
+| **Property**    | `raft.server.log.purge.upto.snapshot.index`                 |
+|-----------------|-------------------------------------------------------------|
+| **Description** | purge logs up to snapshot index when a takes a new snapshot |

Review Comment:
   remove extra `a` ?



##########
ratis-docs/src/site/markdown/configuraions.md:
##########
@@ -0,0 +1,563 @@
+<!---
+  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.
+-->
+
+# Ratis Configuration Reference
+
+## Server Configurations
+
+Most of the server configurations can be found at `RaftServerConfigKeys`.
+To customize configurations, we may 
+1. create a `RaftProperties` object,
+2. set the desired values, and then
+3. pass the customized `RaftProperties` object when building a `RaftServer`.
+
+For example,
+```java
+final RaftProperties properties = new RaftProperties();
+RaftServerConfigKeys.LeaderElection.setPreVote(properties, false);
+final RaftServer raftServer = RaftServer.newBuilder()
+    .setServerId(id)
+    .setStateMachine(stateMachine)
+    .setGroup(raftGroup)
+    .setProperties(properties)
+    .build();
+```
+
+### Server
+
+| **Property**    | `raft.server.storage.dir`                      |
+|-----------------|------------------------------------------------|
+| **Description** | root storage directory to hold RaftServer data |
+| **Type**        | List\<File\>                                   |
+| **Default**     | /tmp/raft-server/                              |
+
+| **Property**    | `raft.server.storage.free-space.min`      |
+|-----------------|-------------------------------------------|
+| **Description** | minimal space requirement for storage dir |
+| **Type**        | SizeInBytes                               |
+| **Default**     | 0MB                                       |
+
+| **Property**    | `raft.server.removed.groups.dir`         |
+|-----------------|------------------------------------------|
+| **Description** | storage directory to hold removed groups |
+| **Type**        | string                                   |
+| **Default**     | /tmp/raft-server/removed-groups/         |
+
+```java
+// GroupManagementApi
+RaftClientReply remove(RaftGroupId groupId,
+    boolean deleteDirectory, boolean renameDirectory) throws IOException;
+```
+When removing an existing group,
+if the `deleteDirectory` flag is set to false and `renameDirectory` is set to true,
+the group data will be renamed to this dir instead of being deleted.
+
+| **Property**    | `raft.server.sleep.deviation.threshold` |
+|:----------------|-----------------------------------------|
+| **Description** | deviation threshold for election sleep  |
+| **Type**        | TimeDuration                            |
+| **Default**     | 300ms                                   |
+
+When a server is a follower,
+it sleeps and wakes up from time to time 
+for checking the heartbeat condition.
+If it cannot receive a heartbeat from the leader
+within the election timeout,
+it starts a leader election.
+
+When a follower server wakes up from a sleep,
+if the actual sleep time is longer
+than the intended sleep time by this threshold,
+it will immediately go back to sleep again,
+instead of checking the heartbeat condition.
+The extra sleep time indicates that
+the server is too busy,
+probably due to GC.
+
+| **Property**    | `raft.server.staging.catchup.gap`  |
+|-----------------|------------------------------------|
+| **Description** | catching up standard of a new peer |
+| **Type**        | int                                |
+| **Default**     | 1000                               |
+
+When bootstrapping a new peer, If the gap between the match index of the
+peer and the leader's latest committed index is less than this gap, we
+treat the peer as caught-up. Increase this number when write throughput is high.
+
+### ThreadPool
+
+Configurations related to RaftServer thread pools.
+
+* proxy thread pool: threads that recover and initialize RaftGroups when RaftServer starts.
+* server thread pool: threads that handle internal RPCs, like appendEntries.
+* client thread pool: threads that handle client requests, like client.io().send().
+
+| **Property**    | `raft.server.threadpool.proxy.cached` |
+|-----------------|---------------------------------------|
+| **Description** | use CachedThreadPool for proxy-pool   |
+| **Type**        | boolean                               |
+| **Default**     | true                                  |
+
+| **Property**    | `raft.server.threadpool.proxy.size` |
+|-----------------|-------------------------------------|
+| **Description** | maximumPoolSize for proxy-pool      |
+| **Type**        | int                                 |
+| **Default**     | 0                                   |
+
+| **Property**    | `raft.server.threadpool.server.cached` |
+|-----------------|----------------------------------------|
+| **Description** | use CachedThreadPool for server-pool   |
+| **Type**        | boolean                                |
+| **Default**     | true                                   |
+
+| **Property**    | `raft.server.threadpool.proxy.size` |
+|-----------------|-------------------------------------|
+| **Description** | maximumPoolSize for server-pool     |
+| **Type**        | int                                 |
+| **Default**     | 0                                   |
+
+| **Property**    | `raft.server.threadpool.client.cached` |
+|-----------------|----------------------------------------|
+| **Description** | use CachedThreadPool for client-pool   |
+| **Type**        | boolean                                |
+| **Default**     | true                                   |
+
+| **Property**    | `raft.server.threadpool.client.size` |
+|-----------------|--------------------------------------|
+| **Description** | maximumPoolSize for client-pool      |
+| **Type**        | int                                  |
+| **Default**     | 0                                    |
+
+### Read
+
+Configurations related to read-only requests.
+
+| **Property**    | `raft.server.read.option` |
+|-----------------|---------------------------|
+| **Description** | read-only option          |
+| **Type**        | enum                      |
+| **Default**     | Option.DEFAULT            |
+
+1. `Option.DEFAULT`: Directly query statemachine. Efficient but may undermine linearizability.
+2. `Option.LINEARIZABLE`: Use ReadIndex (see Raft Paper section 6.4). Maintains linearizability.
+
+| **Property**    | `raft.server.read.timeout`                          |
+|-----------------|-----------------------------------------------------|
+| **Description** | request timeout for linearizable read-only requests |
+| **Type**        | TimeDuration                                        |
+| **Default**     | 10s                                                 |
+
+### Write
+
+Configurations related to write requests.
+
+| **Property**    | `raft.server.write.element-limit` |
+|-----------------|-----------------------------------|
+| **Description** | max pending write requests        |
+| **Type**        | int                               |
+| **Default**     | 4096                              |
+
+| **Property**    | `raft.server.write.byte-limit`      |
+|-----------------|-------------------------------------|
+| **Description** | max pending write requests sum size |
+| **Type**        | SizeInBytes                         |
+| **Default**     | 64MB                                |
+
+Ratis imposes limitations on pending write requests. If the pending requests queue size exceeds element-limit or the
+size of bytes accumulated in pending requests exceeds byte-limit, RaftServer will reject incoming write requests until
+the pending situation is relieved.
+
+| **Property**    | `raft.server.write.follower.gap.ratio.max`                                                                              |
+|-----------------|-------------------------------------------------------------------------------------------------------------------------|
+| **Description** | set a threshold between the majority committed index and slow follower's committed index to guarantee the data in cache |
+| **Type**        | int                                                                                                                     |
+| **Default**     | -1, disable the feature                                                                                                 |
+
+### Watch
+
+Configurations related to watch requests.
+
+| **Property**    | `raft.server.watch.element-limit` |
+|-----------------|-----------------------------------|
+| **Description** | max pending watch requests        |
+| **Type**        | int                               |
+| **Default**     | 65536                             |
+
+| **Property**    | `raft.server.watch.timeout` |
+|-----------------|-----------------------------|
+| **Description** | watch request timeout       |
+| **Type**        | TimeDuration                |
+| **Default**     | 10s                         |
+
+| **Property**    | `raft.server.watch.timeout.denomination`           |
+|-----------------|----------------------------------------------------|
+| **Description** | watch request timeout denomination for rounding up |
+| **Type**        | TimeDuration                                       |
+| **Default**     | 1s                                                 |
+
+watch.timeout should be a multiple of watch.timeout.denomination
+
+### Log
+
+Configurations related to raft log.
+
+| **Property**    | `raft.server.log.use.memory` |
+|-----------------|------------------------------|
+| **Description** | use memory RaftLog           |
+| **Type**        | boolean                      |
+| **Default**     | false                        |
+
+| **Property**    | `raft.server.log.queue.element-limit` |
+|-----------------|---------------------------------------|
+| **Description** | max pending log tasks                 |
+| **Type**        | int                                   |
+| **Default**     | 4096                                  |
+
+| **Property**    | `raft.server.log.queue.byte-limit`             |
+|-----------------|------------------------------------------------|
+| **Description** | size of bytes accumulated in pending log tasks |
+| **Type**        | SizeInBytes                                    |
+| **Default**     | 64MB                                           |
+
+log.queue.byte-limit and log.queue.element-limit are quite similar to write.element-limit and .write.byte-limit. When
+pending IO tasks reached the limit, Ratis will temporarily stall new IO Tasks.
+
+| **Property**    | `raft.server.log.purge.gap`             |
+|-----------------|-----------------------------------------|
+| **Description** | minimal log gap between two purge tasks |
+| **Type**        | int                                     |
+| **Default**     | 1024                                    |
+
+| **Property**    | `raft.server.log.purge.upto.snapshot.index`                 |
+|-----------------|-------------------------------------------------------------|
+| **Description** | purge logs up to snapshot index when a takes a new snapshot |
+| **Type**        | boolean                                                     |
+| **Default**     | false                                                       |
+
+| **Property**    | `raft.server.log.purge.preservation.log.num`                 |
+|-----------------|--------------------------------------------------------------|
+| **Description** | preserve certain logs when purging logs up to snapshot index |
+| **Type**        | long                                                         |
+| **Default**     | 0                                                            |
+
+| **Property**    | `raft.server.log.segment.size.max`          |
+|-----------------|---------------------------------------------|
+| **Description** | max file size for a single Raft Log Segment |
+| **Type**        | SizeInBytes                                 |
+| **Default**     | 8MB                                         |
+
+| **Property**    | `raft.server.log.segment.cache.num.max`                                  |
+|-----------------|--------------------------------------------------------------------------|
+| **Description** | besides the open segment, the max number of segments caching log entries |
+| **Type**        | int                                                                      |
+| **Default**     | 6                                                                        |
+
+| **Property**    | `raft.server.log.segment.cache.size.max`          |
+|-----------------|---------------------------------------------------|
+| **Description** | the max byte size of segments caching log entries |
+| **Type**        | SizeInBytes                                       |
+| **Default**     | 200MB                                             |
+
+| **Property**    | `raft.server.log.preallocated.size` |
+|-----------------|-------------------------------------|
+| **Description** | preallocate size of log segments    |
+| **Type**        | SizeInBytes                         |
+| **Default**     | 4MB                                 |
+
+| **Property**    | `raft.server.log.write.buffer.size`                   |
+|-----------------|-------------------------------------------------------|
+| **Description** | size of direct byte buffer for SegmentLog FileChannel |
+| **Type**        | SizeInBytes                                           |
+| **Default**     | 64KB                                                  |
+
+| **Property**    | `raft.server.log.force.sync.num`                                                |
+|-----------------|---------------------------------------------------------------------------------|
+| **Description** | perform RaftLog flush tasks when pending flush tasks num exceeds force.sync.num |
+| **Type**        | int                                                                             |
+| **Default**     | 128                                                                             |
+
+| **Property**    | `raft.server.log.unsafe-flush.enabled`                                                 |
+|-----------------|----------------------------------------------------------------------------------------|
+| **Description** | unsafe-flush allow increasing flush index without waiting the actual flush to complete |
+| **Type**        | boolean                                                                                |
+| **Default**     | false                                                                                  |
+
+| **Property**    | `raft.server.log.async-flush.enabled`                                       |
+|-----------------|-----------------------------------------------------------------------------|
+| **Description** | async-flush will increase flush index until the actual flush has completed. |
+| **Type**        | boolean                                                                     |
+| **Default**     | false                                                                       |
+
+| **Property**    | `raft.server.log.corruption.policy`     |
+|-----------------|-----------------------------------------|
+| **Description** | the policy to handle corrupted raft log |
+| **Type**        | enum                                    |
+| **Default**     | CorruptionPolicy.EXCEPTION              |
+
+1. `CorruptionPolicy.EXCEPTION`: Rethrow the exception.
+2. `CorruptionPolicy.WARN_AND_RETURN`: Print a warning log message and return all uncorrupted log entries up to the
+   corruption.
+
+#### StateMachineData
+
+configurations related to `StateMachine.DataApi`
+
+| **Property**    | `raft.server.log.statemachine.data.sync`                |
+|-----------------|---------------------------------------------------------|
+| **Description** | RaftLog flush should wait for statemachine data to sync |
+| **Type**        | boolean                                                 |
+| **Default**     | true                                                    |
+
+| **Property**    | `raft.server.log.statemachine.data.sync.timeout` |
+|-----------------|--------------------------------------------------|
+| **Description** | max timeout for statemachine data sync           |
+| **Type**        | TimeDuration                                     |
+| **Default**     | 10s                                              |
+
+| **Property**    | `raft.server.log.statemachine.data.sync.timeout.retry` |
+|-----------------|--------------------------------------------------------|
+| **Description** | retry policy when statemachine data sync timeouts      |
+| **Type**        | int                                                    |
+| **Default**     | -1                                                     |
+
+* -1: retry indefinitely
+* 0: no retry
+* \>0: the number of retries
+
+| **Property**    | `raft.server.log.statemachine.data.read.timeout`         |
+|-----------------|----------------------------------------------------------|
+| **Description** | statemachine data read timeout when get entire log entry |
+| **Type**        | TimeDuration                                             |
+| **Default**     | 1000ms                                                   |
+
+| **Property**    | `raft.server.log.statemachine.data.caching.enabled` |
+|-----------------|-----------------------------------------------------|
+| **Description** | enable RaftLogCache to cache statemachine data      |
+| **Type**        | boolean                                             |
+| **Default**     | false                                               |
+
+If disabled, The stateMachineData will be cached inside the StateMachine itself. RaftLogCache will remove the state
+machine data part when caching a LogEntry.
+
+#### Appender
+
+Configurations related to leader's LogAppender
+
+| **Property**    | `raft.server.log.appender.buffer.element-limit` |
+|-----------------|-------------------------------------------------|
+| **Description** | limits on AppendEntries RPC log entries         |
+| **Type**        | int                                             |
+| **Default**     | 0, means no limit                               |
+
+| **Property**    | `raft.server.log.appender.buffer.byte-limit`                  |
+|-----------------|---------------------------------------------------------------|
+| **Description** | limits on byte size of a single AppendEntries RPC log entries |
+| **Type**        | SizeInBytes                                                   |
+| **Default**     | 4MB                                                           |
+
+It is the limit of
+* max serialized size of a single Log Entry. 
+* max payload of a single AppendEntries RPC.
+
+| **Property**    | `raft.server.log.appender.snapshot.chunk.size.max`                       |
+|-----------------|--------------------------------------------------------------------------|
+| **Description** | max chunk size of the snapshot contained in a single InstallSnapshot RPC |
+| **Type**        | SizeInBytes                                                              |
+| **Default**     | 16MB                                                                     |
+
+| **Property**    | `raft.server.log.appender.install.snapshot.enabled` |
+|-----------------|-----------------------------------------------------|
+| **Description** | allow leader to send snapshot to followers          |
+| **Type**        | boolean                                             |
+| **Default**     | true                                                |
+
+If install.snapshot.enabled is disabled, leader won't send the snapshot to follower. Instead, if the leader detects that
+it does not contain the missing logs of a follower, it will just send a notification to that follower, and the
+follower's statemachine is responsible for fetching and installing snapshot from leader statemachine.
+
+| **Property**    | `raft.server.log.appender.wait-time.min`     |
+|-----------------|----------------------------------------------|
+| **Description** | wait time between 2 subsequent AppendEntries |
+| **Type**        | TimeDuration                                 |
+| **Default**     | 10ms                                         |
+
+### Snapshot
+
+Configurations related to snapshot.
+
+| **Property**    | `raft.server.snapshot.auto.trigger.enabled`                             |
+|-----------------|-------------------------------------------------------------------------|
+| **Description** | whether trigger snapshot when log size exceeds limit                    |
+| **Type**        | boolean                                                                 |
+| **Default**     | false, by default let the state machine to decide when to do checkpoint |
+
+| **Property**    | `raft.server.snapshot.creation.gap`                  |
+|-----------------|------------------------------------------------------|
+| **Description** | the log index gap between to two snapshot creations. |
+| **Type**        | long                                                 |
+| **Default**     | 1024                                                 |
+
+| **Property**    | `raft.server.snapshot.auto.trigger.threshold`                        |
+|-----------------|----------------------------------------------------------------------|
+| **Description** | log size limit (in number of log entries) that triggers the snapshot |
+| **Type**        | long                                                                 |
+| **Default**     | 400000                                                               |
+
+| **Property**    | `raft.server.snapshot.retention.file.num` |
+|-----------------|-------------------------------------------|
+| **Description** | retentive how many old snapshot versions  |
+| **Type**        | int                                       |
+| **Default**     | -1, means only keep latest snapshot       |
+
+### DataStream
+
+ThreadPool configurations related to DataStream Api.
+
+| **Property**    | `raft.server.data-stream.async.request.thread.pool.cached` |
+|-----------------|------------------------------------------------------------|
+| **Description** | use CachedThreadPool for async request pool                |
+| **Type**        | boolean                                                    |
+| **Default**     | false                                                      |
+
+| **Property**    | `raft.server.data-stream.async.request.thread.pool.size` |
+|-----------------|----------------------------------------------------------|
+| **Description** | maximumPoolSize for async request pool                   |
+| **Type**        | int                                                      |
+| **Default**     | 32                                                       |
+
+| **Property**    | `raft.server.data-stream.async.write.thread.pool.cached` |
+|-----------------|----------------------------------------------------------|
+| **Description** | use CachedThreadPool for async write pool                |
+| **Type**        | boolean                                                  |
+| **Default**     | false                                                    |
+
+| **Property**    | `raft.server.data-stream.async.write.thread.pool.size` |
+|-----------------|--------------------------------------------------------|
+| **Description** | maximumPoolSize for async write pool                   |
+| **Type**        | int                                                    |
+| **Default**     | 16                                                     |
+
+| **Property**    | `raft.server.data-stream.client.pool.size`  |
+|-----------------|---------------------------------------------|
+| **Description** | maximumPoolSize for data stream client pool |
+| **Type**        | int                                         |
+| **Default**     | 10                                          |
+
+
+### RPC
+
+Configurations related to Server RPC timeout.
+
+| **Property**    | `raft.server.rpc.timeout.min`      |

Review Comment:
   How about move these propeties related to election timeout to the `LeaderElection` chapter



-- 
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.

To unsubscribe, e-mail: issues-unsubscribe@ratis.apache.org

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