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 a055ed5  RATIS-1233. Move the StateMachineData methods from 
ServerProtoUtils to LogProtoUtils (#350)
a055ed5 is described below

commit a055ed52e77e144a1adc9e89449275f82b36682a
Author: Tsz-Wo Nicholas Sze <[email protected]>
AuthorDate: Sun Dec 13 17:25:44 2020 +0800

    RATIS-1233. Move the StateMachineData methods from ServerProtoUtils to 
LogProtoUtils (#350)
    
    * RATIS-1233. Move the StateMachineData methods from ServerProtoUtils to 
LogProtoUtils.
    
    * Fix checkstyle.
---
 .../apache/ratis/server/impl/ServerProtoUtils.java | 117 ---------------------
 .../apache/ratis/server/raftlog/LogProtoUtils.java | 107 ++++++++++++++++++-
 .../org/apache/ratis/server/raftlog/RaftLog.java   |   7 +-
 .../ratis/server/raftlog/segmented/LogSegment.java |   5 +-
 .../server/raftlog/segmented/SegmentedRaftLog.java |   5 +-
 .../raftlog/segmented/SegmentedRaftLogWorker.java  |   5 +-
 .../apache/ratis/statemachine/StateMachine.java    |   4 +-
 .../ratis/statemachine/TransactionContext.java     |   6 +-
 .../statemachine/impl/TransactionContextImpl.java  |   5 +-
 .../test/java/org/apache/ratis/RaftTestUtil.java   |   3 +-
 10 files changed, 123 insertions(+), 141 deletions(-)

diff --git 
a/ratis-server/src/main/java/org/apache/ratis/server/impl/ServerProtoUtils.java 
b/ratis-server/src/main/java/org/apache/ratis/server/impl/ServerProtoUtils.java
index a2ac071..dc13a0f 100644
--- 
a/ratis-server/src/main/java/org/apache/ratis/server/impl/ServerProtoUtils.java
+++ 
b/ratis-server/src/main/java/org/apache/ratis/server/impl/ServerProtoUtils.java
@@ -20,22 +20,17 @@ package org.apache.ratis.server.impl;
 import org.apache.ratis.client.impl.ClientProtoUtils;
 import org.apache.ratis.proto.RaftProtos.*;
 import org.apache.ratis.proto.RaftProtos.AppendEntriesReplyProto.AppendResult;
-import org.apache.ratis.protocol.ClientId;
-import org.apache.ratis.protocol.RaftClientRequest;
 import org.apache.ratis.protocol.RaftGroupMemberId;
 import org.apache.ratis.protocol.RaftPeer;
 import org.apache.ratis.protocol.RaftPeerId;
 import org.apache.ratis.server.RaftConfiguration;
 import org.apache.ratis.server.protocol.TermIndex;
 import org.apache.ratis.server.raftlog.LogProtoUtils;
-import org.apache.ratis.thirdparty.com.google.protobuf.ByteString;
 import org.apache.ratis.util.Preconditions;
 import org.apache.ratis.util.ProtoUtils;
 
 import java.util.Collection;
 import java.util.List;
-import java.util.Optional;
-import java.util.function.Function;
 
 /** Server proto utilities for internal use. */
 public interface ServerProtoUtils {
@@ -55,27 +50,10 @@ public interface ServerProtoUtils {
         TermIndex.newTermIndex(entry.getTerm(), entry.getIndex());
   }
 
-  static String toTermIndexString(LogEntryProto entry) {
-    return TermIndex.toString(entry.getTerm(), entry.getIndex());
-  }
-
   static String toTermIndexString(TermIndexProto proto) {
     return TermIndex.toString(proto.getTerm(), proto.getIndex());
   }
 
-  static String toStateMachineLogEntryString(StateMachineLogEntryProto smLog,
-                                             
Function<StateMachineLogEntryProto, String> function) {
-    final ByteString clientId = smLog.getClientId();
-    String callIdString = (clientId.isEmpty() ? "<empty clientId>" : 
ClientId.valueOf(clientId))
-        + ", cid=" + smLog.getCallId();
-
-    String smString = "";
-    if (function != null) {
-      smString = "\n\t State Machine: " + function.apply(smLog);
-    }
-    return callIdString + smString;
-  }
-
   static String toShortString(List<LogEntryProto> entries) {
     return entries.size() == 0? "<empty>"
         : "size=" + entries.size() + ", first=" + 
LogProtoUtils.toLogEntryString(entries.get(0));
@@ -156,101 +134,6 @@ public interface ServerProtoUtils {
     return b.build();
   }
 
-  static StateMachineEntryProto.Builder 
toStateMachineEntryProtoBuilder(ByteString stateMachineData) {
-    return 
StateMachineEntryProto.newBuilder().setStateMachineData(stateMachineData);
-  }
-
-  static StateMachineEntryProto.Builder toStateMachineEntryProtoBuilder(int 
logEntryProtoSerializedSize) {
-    return 
StateMachineEntryProto.newBuilder().setLogEntryProtoSerializedSize(logEntryProtoSerializedSize);
-  }
-
-  static StateMachineLogEntryProto toStateMachineLogEntryProto(
-      RaftClientRequest request, ByteString logData, ByteString 
stateMachineData) {
-    if (logData == null) {
-      logData = request.getMessage().getContent();
-    }
-    return toStateMachineLogEntryProto(request.getClientId(), 
request.getCallId(),
-        toStateMachineLogEntryProtoType(request.getType().getTypeCase()), 
logData, stateMachineData);
-  }
-
-  static StateMachineLogEntryProto.Type 
toStateMachineLogEntryProtoType(RaftClientRequestProto.TypeCase typeCase) {
-    switch (typeCase) {
-      case WRITE: return StateMachineLogEntryProto.Type.WRITE;
-      case DATASTREAM: return StateMachineLogEntryProto.Type.DATASTREAM;
-      default:
-        throw new IllegalStateException("Unexpected type case " + typeCase);
-    }
-  }
-
-  static StateMachineLogEntryProto toStateMachineLogEntryProto(ClientId 
clientId, long callId,
-      StateMachineLogEntryProto.Type type, ByteString logData, ByteString 
stateMachineData) {
-    final StateMachineLogEntryProto.Builder b = 
StateMachineLogEntryProto.newBuilder()
-        .setClientId(clientId.toByteString())
-        .setCallId(callId)
-        .setType(type)
-        .setLogData(logData);
-    if (stateMachineData != null) {
-      
b.setStateMachineEntry(toStateMachineEntryProtoBuilder(stateMachineData));
-    }
-    return b.build();
-  }
-
-  static Optional<StateMachineEntryProto> getStateMachineEntry(LogEntryProto 
entry) {
-    return Optional.of(entry)
-        .filter(LogEntryProto::hasStateMachineLogEntry)
-        .map(LogEntryProto::getStateMachineLogEntry)
-        .filter(StateMachineLogEntryProto::hasStateMachineEntry)
-        .map(StateMachineLogEntryProto::getStateMachineEntry);
-  }
-
-  static Optional<ByteString> getStateMachineData(LogEntryProto entry) {
-    return getStateMachineEntry(entry)
-        .map(StateMachineEntryProto::getStateMachineData);
-  }
-
-  static boolean shouldReadStateMachineData(LogEntryProto entry) {
-    return getStateMachineData(entry).map(ByteString::isEmpty).orElse(false);
-  }
-
-  /**
-   * If the given entry has state machine log entry and it has state machine 
data,
-   * build a new entry without the state machine data.
-   *
-   * @return a new entry without the state machine data if the given has state 
machine data;
-   *         otherwise, return the given entry.
-   */
-  static LogEntryProto removeStateMachineData(LogEntryProto entry) {
-    return getStateMachineData(entry)
-        .filter(stateMachineData -> !stateMachineData.isEmpty())
-        .map(_dummy -> rebuildLogEntryProto(entry, 
toStateMachineEntryProtoBuilder(entry.getSerializedSize())))
-        .orElse(entry);
-  }
-
-  static LogEntryProto rebuildLogEntryProto(LogEntryProto entry, 
StateMachineEntryProto.Builder smEntry) {
-    return LogEntryProto.newBuilder(entry).setStateMachineLogEntry(
-        
StateMachineLogEntryProto.newBuilder(entry.getStateMachineLogEntry()).setStateMachineEntry(smEntry)
-    ).build();
-  }
-
-  /**
-   * Return a new log entry based on the input log entry with stateMachineData 
added.
-   * @param stateMachineData - state machine data to be added
-   * @param entry - log entry to which stateMachineData needs to be added
-   * @return LogEntryProto with stateMachineData added
-   */
-  static LogEntryProto addStateMachineData(ByteString stateMachineData, 
LogEntryProto entry) {
-    Preconditions.assertTrue(shouldReadStateMachineData(entry),
-        () -> "Failed to addStateMachineData to " + entry + " since 
shouldReadStateMachineData is false.");
-    return rebuildLogEntryProto(entry, 
toStateMachineEntryProtoBuilder(stateMachineData));
-  }
-
-  static int getSerializedSize(LogEntryProto entry) {
-    return getStateMachineEntry(entry)
-        .filter(smEnty -> smEnty.getStateMachineData().isEmpty())
-        .map(StateMachineEntryProto::getLogEntryProtoSerializedSize)
-        .orElseGet(entry::getSerializedSize);
-  }
-
   static RaftRpcReplyProto.Builder toRaftRpcReplyProtoBuilder(
       RaftPeerId requestorId, RaftGroupMemberId replyId, boolean success) {
     return ClientProtoUtils.toRaftRpcReplyProtoBuilder(
diff --git 
a/ratis-server/src/main/java/org/apache/ratis/server/raftlog/LogProtoUtils.java 
b/ratis-server/src/main/java/org/apache/ratis/server/raftlog/LogProtoUtils.java
index a3cd5cf..ff7a027 100644
--- 
a/ratis-server/src/main/java/org/apache/ratis/server/raftlog/LogProtoUtils.java
+++ 
b/ratis-server/src/main/java/org/apache/ratis/server/raftlog/LogProtoUtils.java
@@ -18,9 +18,12 @@
 package org.apache.ratis.server.raftlog;
 
 import org.apache.ratis.proto.RaftProtos.*;
+import org.apache.ratis.protocol.ClientId;
+import org.apache.ratis.protocol.RaftClientRequest;
 import org.apache.ratis.server.RaftConfiguration;
-import org.apache.ratis.server.impl.ServerProtoUtils;
 import org.apache.ratis.server.protocol.TermIndex;
+import org.apache.ratis.thirdparty.com.google.protobuf.ByteString;
+import org.apache.ratis.util.Preconditions;
 import org.apache.ratis.util.ProtoUtils;
 
 import java.util.Arrays;
@@ -32,13 +35,21 @@ import java.util.stream.Collectors;
 public final class LogProtoUtils {
   private LogProtoUtils() {}
 
+  public static String toStateMachineLogEntryString(StateMachineLogEntryProto 
proto,
+      Function<StateMachineLogEntryProto, String> function) {
+    final ByteString clientId = proto.getClientId();
+    return (clientId.isEmpty() ? "<empty ClientId>" : 
ClientId.valueOf(clientId))
+        + ",cid=" + proto.getCallId()
+        + Optional.ofNullable(function).map(f -> f.apply(proto)).map(s -> "," 
+ s).orElse("");
+  }
+
   public static String toLogEntryString(LogEntryProto entry, 
Function<StateMachineLogEntryProto, String> function) {
     if (entry == null) {
       return null;
     }
     final String s;
     if (entry.hasStateMachineLogEntry()) {
-      s = ", " + 
ServerProtoUtils.toStateMachineLogEntryString(entry.getStateMachineLogEntry(), 
function);
+      s = ", " + toStateMachineLogEntryString(entry.getStateMachineLogEntry(), 
function);
     } else if (entry.hasMetadataEntry()) {
       final MetadataProto metadata = entry.getMetadataEntry();
       s = "(c:" + metadata.getCommitIndex() + ")";
@@ -88,4 +99,96 @@ public final class LogProtoUtils {
         
.setMetadataEntry(MetadataProto.newBuilder().setCommitIndex(commitIndex))
         .build();
   }
+
+  /**
+   * If the given entry has state machine log entry and it has state machine 
data,
+   * build a new entry without the state machine data.
+   *
+   * @return a new entry without the state machine data if the given has state 
machine data;
+   *         otherwise, return the given entry.
+   */
+  public static LogEntryProto removeStateMachineData(LogEntryProto entry) {
+    return getStateMachineEntry(entry)
+        .map(StateMachineEntryProto::getStateMachineData)
+        .filter(stateMachineData -> !stateMachineData.isEmpty())
+        .map(_dummy -> replaceStateMachineDataWithSerializedSize(entry))
+        .orElse(entry);
+  }
+
+  private static LogEntryProto 
replaceStateMachineDataWithSerializedSize(LogEntryProto entry) {
+    return replaceStateMachineEntry(entry,
+        
StateMachineEntryProto.newBuilder().setLogEntryProtoSerializedSize(entry.getSerializedSize()));
+  }
+
+  private static LogEntryProto replaceStateMachineEntry(LogEntryProto proto, 
StateMachineEntryProto.Builder newEntry) {
+    Preconditions.assertTrue(proto.hasStateMachineLogEntry(), () -> 
"Unexpected proto " + proto);
+    return LogEntryProto.newBuilder(proto).setStateMachineLogEntry(
+        
StateMachineLogEntryProto.newBuilder(proto.getStateMachineLogEntry()).setStateMachineEntry(newEntry)
+    ).build();
+  }
+
+  /**
+   * Return a new log entry based on the input log entry with stateMachineData 
added.
+   * @param stateMachineData - state machine data to be added
+   * @param entry - log entry to which stateMachineData needs to be added
+   * @return LogEntryProto with stateMachineData added
+   */
+  static LogEntryProto addStateMachineData(ByteString stateMachineData, 
LogEntryProto entry) {
+    Preconditions.assertTrue(isStateMachineDataEmpty(entry),
+        () -> "Failed to addStateMachineData to " + entry + " since 
shouldReadStateMachineData is false.");
+    return replaceStateMachineEntry(entry, 
StateMachineEntryProto.newBuilder().setStateMachineData(stateMachineData));
+  }
+
+  public static boolean isStateMachineDataEmpty(LogEntryProto entry) {
+    return getStateMachineEntry(entry)
+        .map(StateMachineEntryProto::getStateMachineData)
+        .map(ByteString::isEmpty)
+        .orElse(false);
+  }
+
+  private static Optional<StateMachineEntryProto> 
getStateMachineEntry(LogEntryProto entry) {
+    return Optional.of(entry)
+        .filter(LogEntryProto::hasStateMachineLogEntry)
+        .map(LogEntryProto::getStateMachineLogEntry)
+        .filter(StateMachineLogEntryProto::hasStateMachineEntry)
+        .map(StateMachineLogEntryProto::getStateMachineEntry);
+  }
+
+  public static int getSerializedSize(LogEntryProto entry) {
+    return getStateMachineEntry(entry)
+        .filter(stateMachineEntry -> 
stateMachineEntry.getStateMachineData().isEmpty())
+        .map(StateMachineEntryProto::getLogEntryProtoSerializedSize)
+        .orElseGet(entry::getSerializedSize);
+  }
+
+  private static StateMachineLogEntryProto.Type 
toStateMachineLogEntryProtoType(RaftClientRequestProto.TypeCase type) {
+    switch (type) {
+      case WRITE: return StateMachineLogEntryProto.Type.WRITE;
+      case DATASTREAM: return StateMachineLogEntryProto.Type.DATASTREAM;
+      default:
+        throw new IllegalStateException("Unexpected request type " + type);
+    }
+  }
+
+  public static StateMachineLogEntryProto toStateMachineLogEntryProto(
+      RaftClientRequest request, ByteString logData, ByteString 
stateMachineData) {
+    if (logData == null) {
+      logData = request.getMessage().getContent();
+    }
+    final StateMachineLogEntryProto.Type type = 
toStateMachineLogEntryProtoType(request.getType().getTypeCase());
+    return toStateMachineLogEntryProto(request.getClientId(), 
request.getCallId(), type, logData, stateMachineData);
+  }
+
+  public static StateMachineLogEntryProto toStateMachineLogEntryProto(ClientId 
clientId, long callId,
+      StateMachineLogEntryProto.Type type, ByteString logData, ByteString 
stateMachineData) {
+    final StateMachineLogEntryProto.Builder b = 
StateMachineLogEntryProto.newBuilder()
+        .setClientId(clientId.toByteString())
+        .setCallId(callId)
+        .setType(type)
+        .setLogData(logData);
+    Optional.ofNullable(stateMachineData)
+        .map(StateMachineEntryProto.newBuilder()::setStateMachineData)
+        .ifPresent(b::setStateMachineEntry);
+    return b.build();
+  }
 }
diff --git 
a/ratis-server/src/main/java/org/apache/ratis/server/raftlog/RaftLog.java 
b/ratis-server/src/main/java/org/apache/ratis/server/raftlog/RaftLog.java
index dd2a27a..a401107 100644
--- a/ratis-server/src/main/java/org/apache/ratis/server/raftlog/RaftLog.java
+++ b/ratis-server/src/main/java/org/apache/ratis/server/raftlog/RaftLog.java
@@ -24,7 +24,6 @@ import org.apache.ratis.protocol.RaftPeerId;
 import org.apache.ratis.protocol.exceptions.StateMachineException;
 import org.apache.ratis.server.RaftServerConfigKeys;
 import org.apache.ratis.server.RaftConfiguration;
-import org.apache.ratis.server.impl.ServerProtoUtils;
 import org.apache.ratis.server.metrics.RaftLogMetrics;
 import org.apache.ratis.server.protocol.TermIndex;
 import org.apache.ratis.statemachine.TransactionContext;
@@ -512,7 +511,7 @@ public abstract class RaftLog implements 
RaftLogSequentialOps, Closeable {
     }
 
     public int getSerializedSize() {
-      return ServerProtoUtils.getSerializedSize(logEntry);
+      return LogProtoUtils.getSerializedSize(logEntry);
     }
 
     public LogEntryProto getEntry(TimeDuration timeout) throws 
RaftLogIOException, TimeoutException {
@@ -522,7 +521,7 @@ public abstract class RaftLog implements 
RaftLogSequentialOps, Closeable {
       }
 
       try {
-        entryProto = future.thenApply(data -> 
ServerProtoUtils.addStateMachineData(data, logEntry))
+        entryProto = future.thenApply(data -> 
LogProtoUtils.addStateMachineData(data, logEntry))
             .get(timeout.getDuration(), timeout.getUnit());
       } catch (TimeoutException t) {
         final String err = getName() + ": Timeout readStateMachineData for " + 
toLogEntryString(logEntry);
@@ -535,7 +534,7 @@ public abstract class RaftLog implements 
RaftLogSequentialOps, Closeable {
       }
       // by this time we have already read the state machine data,
       // so the log entry data should be set now
-      if (ServerProtoUtils.shouldReadStateMachineData(entryProto)) {
+      if (LogProtoUtils.isStateMachineDataEmpty(entryProto)) {
         final String err = getName() + ": State machine data not set for " + 
toLogEntryString(logEntry);
         LOG.error(err);
         throw new RaftLogIOException(err);
diff --git 
a/ratis-server/src/main/java/org/apache/ratis/server/raftlog/segmented/LogSegment.java
 
b/ratis-server/src/main/java/org/apache/ratis/server/raftlog/segmented/LogSegment.java
index 0510b14..30249bb 100644
--- 
a/ratis-server/src/main/java/org/apache/ratis/server/raftlog/segmented/LogSegment.java
+++ 
b/ratis-server/src/main/java/org/apache/ratis/server/raftlog/segmented/LogSegment.java
@@ -22,6 +22,7 @@ import 
org.apache.ratis.server.RaftServerConfigKeys.Log.CorruptionPolicy;
 import org.apache.ratis.server.impl.ServerProtoUtils;
 import org.apache.ratis.server.metrics.RaftLogMetrics;
 import org.apache.ratis.server.protocol.TermIndex;
+import org.apache.ratis.server.raftlog.LogProtoUtils;
 import org.apache.ratis.server.raftlog.RaftLogIOException;
 import org.apache.ratis.server.storage.RaftStorage;
 import 
org.apache.ratis.thirdparty.com.google.common.annotations.VisibleForTesting;
@@ -73,9 +74,9 @@ public class LogSegment implements Comparable<Long> {
   static long getEntrySize(LogEntryProto entry, Op op) {
     LogEntryProto e = entry;
     if (op == Op.CHECK_SEGMENT_FILE_FULL) {
-      e = ServerProtoUtils.removeStateMachineData(entry);
+      e = LogProtoUtils.removeStateMachineData(entry);
     } else if (op == Op.LOAD_SEGMENT_FILE || op == 
Op.WRITE_CACHE_WITH_STATE_MACHINE_CACHE) {
-      Preconditions.assertTrue(entry == 
ServerProtoUtils.removeStateMachineData(entry),
+      Preconditions.assertTrue(entry == 
LogProtoUtils.removeStateMachineData(entry),
           () -> "Unexpected LogEntryProto with StateMachine data: op=" + op + 
", entry=" + entry);
     } else {
       Preconditions.assertTrue(op == 
Op.WRITE_CACHE_WITHOUT_STATE_MACHINE_CACHE || op == Op.REMOVE_CACHE,
diff --git 
a/ratis-server/src/main/java/org/apache/ratis/server/raftlog/segmented/SegmentedRaftLog.java
 
b/ratis-server/src/main/java/org/apache/ratis/server/raftlog/segmented/SegmentedRaftLog.java
index d37cccf..df3fe61 100644
--- 
a/ratis-server/src/main/java/org/apache/ratis/server/raftlog/segmented/SegmentedRaftLog.java
+++ 
b/ratis-server/src/main/java/org/apache/ratis/server/raftlog/segmented/SegmentedRaftLog.java
@@ -22,7 +22,6 @@ import org.apache.ratis.protocol.RaftGroupMemberId;
 import org.apache.ratis.protocol.RaftPeerId;
 import org.apache.ratis.server.RaftServer;
 import org.apache.ratis.server.RaftServerConfigKeys;
-import org.apache.ratis.server.impl.ServerProtoUtils;
 import org.apache.ratis.server.protocol.TermIndex;
 import org.apache.ratis.server.raftlog.LogProtoUtils;
 import org.apache.ratis.server.raftlog.RaftLog;
@@ -283,7 +282,7 @@ public class SegmentedRaftLog extends RaftLog {
     if (entry == null) {
       throw new RaftLogIOException("Log entry not found: index = " + index);
     }
-    if (!ServerProtoUtils.shouldReadStateMachineData(entry)) {
+    if (!LogProtoUtils.isStateMachineDataEmpty(entry)) {
       return new EntryWithData(entry, null);
     }
 
@@ -404,7 +403,7 @@ public class SegmentedRaftLog extends RaftLog {
           fileLogWorker.writeLogEntry(entry).getFuture();
       if (stateMachineCachingEnabled) {
         // The stateMachineData will be cached inside the StateMachine itself.
-        cache.appendEntry(ServerProtoUtils.removeStateMachineData(entry),
+        cache.appendEntry(LogProtoUtils.removeStateMachineData(entry),
             LogSegment.Op.WRITE_CACHE_WITH_STATE_MACHINE_CACHE);
       } else {
         cache.appendEntry(entry, 
LogSegment.Op.WRITE_CACHE_WITHOUT_STATE_MACHINE_CACHE);
diff --git 
a/ratis-server/src/main/java/org/apache/ratis/server/raftlog/segmented/SegmentedRaftLogWorker.java
 
b/ratis-server/src/main/java/org/apache/ratis/server/raftlog/segmented/SegmentedRaftLogWorker.java
index 9c99f7d..5c238bc 100644
--- 
a/ratis-server/src/main/java/org/apache/ratis/server/raftlog/segmented/SegmentedRaftLogWorker.java
+++ 
b/ratis-server/src/main/java/org/apache/ratis/server/raftlog/segmented/SegmentedRaftLogWorker.java
@@ -26,7 +26,6 @@ import org.apache.ratis.protocol.RaftGroupMemberId;
 import org.apache.ratis.protocol.exceptions.TimeoutIOException;
 import org.apache.ratis.server.RaftServer;
 import org.apache.ratis.server.RaftServerConfigKeys;
-import org.apache.ratis.server.impl.ServerProtoUtils;
 import org.apache.ratis.server.metrics.RaftLogMetrics;
 import org.apache.ratis.server.raftlog.LogProtoUtils;
 import org.apache.ratis.server.raftlog.RaftLog;
@@ -454,7 +453,7 @@ class SegmentedRaftLogWorker {
     private final CompletableFuture<Long> combined;
 
     WriteLog(LogEntryProto entry) {
-      this.entry = ServerProtoUtils.removeStateMachineData(entry);
+      this.entry = LogProtoUtils.removeStateMachineData(entry);
       if (this.entry == entry) {
         final StateMachineLogEntryProto proto = 
entry.hasStateMachineLogEntry()? entry.getStateMachineLogEntry(): null;
         if (stateMachine != null && proto != null && proto.getType() == 
StateMachineLogEntryProto.Type.DATASTREAM) {
@@ -487,7 +486,7 @@ class SegmentedRaftLogWorker {
 
     @Override
     int getSerializedSize() {
-      return ServerProtoUtils.getSerializedSize(entry);
+      return LogProtoUtils.getSerializedSize(entry);
     }
 
     @Override
diff --git 
a/ratis-server/src/main/java/org/apache/ratis/statemachine/StateMachine.java 
b/ratis-server/src/main/java/org/apache/ratis/statemachine/StateMachine.java
index a950167..c454392 100644
--- a/ratis-server/src/main/java/org/apache/ratis/statemachine/StateMachine.java
+++ b/ratis-server/src/main/java/org/apache/ratis/statemachine/StateMachine.java
@@ -27,8 +27,8 @@ import org.apache.ratis.protocol.RaftGroupId;
 import org.apache.ratis.protocol.RaftGroupMemberId;
 import org.apache.ratis.protocol.RaftPeerId;
 import org.apache.ratis.server.RaftServer;
-import org.apache.ratis.server.impl.ServerProtoUtils;
 import org.apache.ratis.server.protocol.TermIndex;
+import org.apache.ratis.server.raftlog.LogProtoUtils;
 import org.apache.ratis.server.storage.RaftStorage;
 import org.apache.ratis.thirdparty.com.google.protobuf.ByteString;
 import org.apache.ratis.util.LifeCycle;
@@ -456,6 +456,6 @@ public interface StateMachine extends Closeable {
    * @return the string representation of the proto.
    */
   default String 
toStateMachineLogEntryString(RaftProtos.StateMachineLogEntryProto proto) {
-    return ServerProtoUtils.toStateMachineLogEntryString(proto, null);
+    return LogProtoUtils.toStateMachineLogEntryString(proto, null);
   }
 }
diff --git 
a/ratis-server/src/main/java/org/apache/ratis/statemachine/TransactionContext.java
 
b/ratis-server/src/main/java/org/apache/ratis/statemachine/TransactionContext.java
index 17738e3..b6ed26b 100644
--- 
a/ratis-server/src/main/java/org/apache/ratis/statemachine/TransactionContext.java
+++ 
b/ratis-server/src/main/java/org/apache/ratis/statemachine/TransactionContext.java
@@ -1,4 +1,4 @@
-/**
+/*
  * 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
@@ -21,7 +21,7 @@ import org.apache.ratis.proto.RaftProtos.LogEntryProto;
 import org.apache.ratis.proto.RaftProtos.RaftPeerRole;
 import org.apache.ratis.proto.RaftProtos.StateMachineLogEntryProto;
 import org.apache.ratis.protocol.RaftClientRequest;
-import org.apache.ratis.server.impl.ServerProtoUtils;
+import org.apache.ratis.server.raftlog.LogProtoUtils;
 import org.apache.ratis.statemachine.impl.TransactionContextImpl;
 import org.apache.ratis.thirdparty.com.google.protobuf.ByteString;
 import org.apache.ratis.util.Preconditions;
@@ -200,7 +200,7 @@ public interface TransactionContext {
             () -> "serverRole MUST be LEADER since clientRequest != null, 
serverRole is " + serverRole);
         Preconditions.assertNull(logEntry, () -> "logEntry MUST be null since 
clientRequest != null");
         if (stateMachineLogEntry == null) {
-          stateMachineLogEntry = 
ServerProtoUtils.toStateMachineLogEntryProto(clientRequest, logData, 
stateMachineData);
+          stateMachineLogEntry = 
LogProtoUtils.toStateMachineLogEntryProto(clientRequest, logData, 
stateMachineData);
         }
         return new TransactionContextImpl(stateMachine, clientRequest, 
stateMachineLogEntry, stateMachineContext);
       } else {
diff --git 
a/ratis-server/src/main/java/org/apache/ratis/statemachine/impl/TransactionContextImpl.java
 
b/ratis-server/src/main/java/org/apache/ratis/statemachine/impl/TransactionContextImpl.java
index 86206de..c1a72c6 100644
--- 
a/ratis-server/src/main/java/org/apache/ratis/statemachine/impl/TransactionContextImpl.java
+++ 
b/ratis-server/src/main/java/org/apache/ratis/statemachine/impl/TransactionContextImpl.java
@@ -1,4 +1,4 @@
-/**
+/*
  * 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
@@ -21,7 +21,6 @@ import org.apache.ratis.proto.RaftProtos.LogEntryProto;
 import org.apache.ratis.proto.RaftProtos.RaftPeerRole;
 import org.apache.ratis.proto.RaftProtos.StateMachineLogEntryProto;
 import org.apache.ratis.protocol.RaftClientRequest;
-import org.apache.ratis.server.impl.ServerProtoUtils;
 import org.apache.ratis.server.raftlog.LogProtoUtils;
 import org.apache.ratis.statemachine.StateMachine;
 import org.apache.ratis.statemachine.TransactionContext;
@@ -85,7 +84,7 @@ public class TransactionContextImpl implements 
TransactionContext {
     this(RaftPeerRole.LEADER, stateMachine);
     this.clientRequest = clientRequest;
     this.smLogEntryProto = smLogEntryProto != null? smLogEntryProto
-        : ServerProtoUtils.toStateMachineLogEntryProto(clientRequest, null, 
null);
+        : LogProtoUtils.toStateMachineLogEntryProto(clientRequest, null, null);
     this.stateMachineContext = stateMachineContext;
   }
 
diff --git a/ratis-server/src/test/java/org/apache/ratis/RaftTestUtil.java 
b/ratis-server/src/test/java/org/apache/ratis/RaftTestUtil.java
index de3486d..7abddf3 100644
--- a/ratis-server/src/test/java/org/apache/ratis/RaftTestUtil.java
+++ b/ratis-server/src/test/java/org/apache/ratis/RaftTestUtil.java
@@ -31,7 +31,6 @@ import org.apache.ratis.server.RaftServerConfigKeys;
 import org.apache.ratis.server.impl.BlockRequestHandlingInjection;
 import org.apache.ratis.server.impl.DelayLocalExecutionInjection;
 import org.apache.ratis.server.impl.MiniRaftCluster;
-import org.apache.ratis.server.impl.ServerProtoUtils;
 import org.apache.ratis.server.protocol.TermIndex;
 import org.apache.ratis.server.raftlog.LogProtoUtils;
 import org.apache.ratis.server.raftlog.RaftLog;
@@ -363,7 +362,7 @@ public interface RaftTestUtil {
     private SimpleOperation(ClientId clientId, long callId, String op, boolean 
hasStateMachineData) {
       this.op = Objects.requireNonNull(op);
       final ByteString bytes = ProtoUtils.toByteString(op);
-      this.smLogEntryProto = ServerProtoUtils.toStateMachineLogEntryProto(
+      this.smLogEntryProto = LogProtoUtils.toStateMachineLogEntryProto(
           clientId, callId, StateMachineLogEntryProto.Type.WRITE, bytes, 
hasStateMachineData? bytes: null);
     }
 

Reply via email to