Skip to content

Commit 8212ecc

Browse files
committed
address review issues
1 parent a103c33 commit 8212ecc

File tree

3 files changed

+12
-8
lines changed

3 files changed

+12
-8
lines changed

ratis-server/src/main/java/org/apache/ratis/server/impl/RaftServerImpl.java

Lines changed: 7 additions & 3 deletions
Original file line numberDiff line numberDiff line change
@@ -114,6 +114,7 @@
114114
import org.apache.ratis.util.ReferenceCountedObject;
115115
import org.apache.ratis.util.TimeDuration;
116116
import org.apache.ratis.util.function.CheckedSupplier;
117+
import org.apache.ratis.util.function.UncheckedAutoCloseableSupplier;
117118

118119
import java.io.File;
119120
import java.io.IOException;
@@ -1690,10 +1691,13 @@ leaderId, getMemberId(), currentTerm, followerCommit, inconsistencyReplyNextInde
16901691
});
16911692
}
16921693
private CompletableFuture<Void> appendLog(ReferenceCountedObject<List<LogEntryProto>> entriesRef) {
1693-
final List<LogEntryProto> entries = entriesRef.retain();
1694-
final List<ConsecutiveIndices> entriesTermIndices = ConsecutiveIndices.convert(entries);
1695-
appendLogTermIndices.append(entriesTermIndices);
1694+
final List<ConsecutiveIndices> entriesTermIndices;
1695+
try(UncheckedAutoCloseableSupplier<List<LogEntryProto>> entries = entriesRef.retainAndReleaseOnClose()) {
1696+
entriesTermIndices = ConsecutiveIndices.convert(entries.get());
1697+
appendLogTermIndices.append(entriesTermIndices);
1698+
}
16961699

1700+
entriesRef.retain();
16971701
return appendLogFuture.updateAndGet(f -> f.thenCompose(
16981702
ignored -> JavaUtils.allOf(state.getLog().append(entriesRef))))
16991703
.whenComplete((v, e) -> {

ratis-server/src/main/java/org/apache/ratis/server/impl/RaftServerProxy.java

Lines changed: 2 additions & 1 deletion
Original file line numberDiff line numberDiff line change
@@ -653,7 +653,8 @@ public CompletableFuture<AppendEntriesReplyProto> appendEntriesAsync(
653653
try {
654654
final RaftGroupId groupId = ProtoUtils.toRaftGroupId(request.getServerRequest().getRaftGroupId());
655655
return getImplFuture(groupId)
656-
.thenCompose(impl -> impl.executeSubmitServerRequestAsync(() -> impl.appendEntriesAsync(requestRef)));
656+
.thenCompose(impl -> JavaUtils.callAsUnchecked(
657+
() -> impl.appendEntriesAsync(requestRef), CompletionException::new));
657658
} finally {
658659
requestRef.release();
659660
}

ratis-server/src/main/java/org/apache/ratis/server/impl/ServerImplUtils.java

Lines changed: 3 additions & 4 deletions
Original file line numberDiff line numberDiff line change
@@ -138,13 +138,12 @@ synchronized Long getTerm(long index) {
138138

139139
synchronized void append(List<ConsecutiveIndices> entriesTermIndices) {
140140
for(ConsecutiveIndices indices : entriesTermIndices) {
141-
// validate index0
142-
final long index0 = indices.startIndex;
141+
// validate startIndex
143142
final Map.Entry<Long, ConsecutiveIndices> lastEntry = map.lastEntry();
144143
if (lastEntry != null) {
145-
Preconditions.assertSame(lastEntry.getValue().getNextIndex(), index0, "index0");
144+
Preconditions.assertSame(lastEntry.getValue().getNextIndex(), indices.startIndex, "startIndex");
146145
}
147-
map.put(index0, indices);
146+
map.put(indices.startIndex, indices);
148147
}
149148
}
150149

0 commit comments

Comments
 (0)