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 c8d74d056 RATIS-1900. Do not use FileInputStream/FileOutputStream. 
(#946)
c8d74d056 is described below

commit c8d74d056bcbe2ae1dd042bb1e70d044a45cfa00
Author: Tsz-Wo Nicholas Sze <[email protected]>
AuthorDate: Tue Nov 14 00:13:51 2023 -0800

    RATIS-1900. Do not use FileInputStream/FileOutputStream. (#946)
---
 pom.xml                                            | 30 ++++++++++++
 .../java/org/apache/ratis/protocol/RaftId.java     |  4 +-
 .../com/google/protobuf/ByteStringUtils.java       | 30 ------------
 .../arithmetic/ArithmeticStateMachine.java         | 11 ++---
 .../ratis/examples/filestore/cli/DataStream.java   |  8 +--
 .../ratis/examples/filestore/cli/LoadGen.java      |  8 +--
 .../raftlog/segmented/SegmentedRaftLogReader.java  | 25 +++++-----
 .../ratis/server/storage/FileChunkReader.java      | 16 +++---
 .../ratis/server/storage/RaftStorageImpl.java      | 16 +++---
 .../storage/RaftStorageMetadataFileImpl.java       |  4 +-
 .../ratis/server/storage/SnapshotManager.java      | 57 ++++++++++++----------
 .../ratis/datastream/DataStreamClusterTests.java   |  8 +--
 .../ratis/datastream/DataStreamTestUtils.java      |  7 +--
 .../apache/ratis/security/SecurityTestUtils.java   |  5 +-
 .../server/raftlog/segmented/TestLogSegment.java   |  2 +
 .../raftlog/segmented/TestRaftLogReadWrite.java    | 11 +++--
 16 files changed, 132 insertions(+), 110 deletions(-)

diff --git a/pom.xml b/pom.xml
index d0da71e30..f7a9c334c 100644
--- a/pom.xml
+++ b/pom.xml
@@ -184,6 +184,7 @@
     
<build-helper-maven-plugin.version>3.3.0</build-helper-maven-plugin.version>
     <exec-maven-plugin.version>3.1.0</exec-maven-plugin.version>
     <extra-enforcer-rules.version>1.6.1</extra-enforcer-rules.version>
+    
<restrict-imports-enforcer-rules.version>2.4.0</restrict-imports-enforcer-rules.version>
     <license-maven-plugin.version>2.2.0</license-maven-plugin.version>
 
     <protobuf-maven-plugin.version>0.6.1</protobuf-maven-plugin.version>
@@ -529,7 +530,36 @@
               </requireJavaVersion>
             </rules>
           </configuration>
+          <dependencies>
+            <dependency>
+              <groupId>de.skuzzle.enforcer</groupId>
+              <artifactId>restrict-imports-enforcer-rule</artifactId>
+              <version>${restrict-imports-enforcer-rules.version}</version>
+            </dependency>
+          </dependencies>
+          <executions>
+            <execution>
+              <id>bannedImports-finalizers</id>
+              <phase>process-sources</phase>
+              <goals>
+                <goal>enforce</goal>
+              </goals>
+              <configuration>
+                <rules>
+                  <restrictImports>
+                    <includeTestCode>false</includeTestCode>
+                    <reason>Classes (e.g. FileInputStream, FileOutputStream) 
overriding the Object.finalize() method will cause gc to run a much longer 
time. As stated in Effective Java, there is a severe performance penalty for 
using finalizers.</reason>
+                    <bannedImports>
+                      <bannedImport>java.io.FileInputStream</bannedImport>
+                      <bannedImport>java.io.FileOutputStream</bannedImport>
+                    </bannedImports>
+                  </restrictImports>
+                </rules>
+              </configuration>
+            </execution>
+          </executions>
         </plugin>
+
         <plugin>
           <groupId>org.apache.maven.plugins</groupId>
           <artifactId>maven-assembly-plugin</artifactId>
diff --git a/ratis-common/src/main/java/org/apache/ratis/protocol/RaftId.java 
b/ratis-common/src/main/java/org/apache/ratis/protocol/RaftId.java
index f1d07a980..161bd830b 100644
--- a/ratis-common/src/main/java/org/apache/ratis/protocol/RaftId.java
+++ b/ratis-common/src/main/java/org/apache/ratis/protocol/RaftId.java
@@ -20,7 +20,7 @@ package org.apache.ratis.protocol;
 import org.apache.ratis.thirdparty.com.google.common.cache.Cache;
 import org.apache.ratis.thirdparty.com.google.common.cache.CacheBuilder;
 import org.apache.ratis.thirdparty.com.google.protobuf.ByteString;
-import org.apache.ratis.thirdparty.com.google.protobuf.ByteStringUtils;
+import org.apache.ratis.thirdparty.com.google.protobuf.UnsafeByteOperations;
 import org.apache.ratis.util.JavaUtils;
 import org.apache.ratis.util.Preconditions;
 
@@ -49,7 +49,7 @@ public abstract class RaftId {
     ByteBuffer.wrap(array)
         .putLong(uuid.getMostSignificantBits())
         .putLong(uuid.getLeastSignificantBits());
-    return ByteStringUtils.unsafeWrap(array);
+    return UnsafeByteOperations.unsafeWrap(array);
   }
 
   abstract static class Factory<ID extends RaftId> {
diff --git 
a/ratis-common/src/main/java/org/apache/ratis/thirdparty/com/google/protobuf/ByteStringUtils.java
 
b/ratis-common/src/main/java/org/apache/ratis/thirdparty/com/google/protobuf/ByteStringUtils.java
deleted file mode 100644
index 85174979f..000000000
--- 
a/ratis-common/src/main/java/org/apache/ratis/thirdparty/com/google/protobuf/ByteStringUtils.java
+++ /dev/null
@@ -1,30 +0,0 @@
-/*
- * 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.
- */
-package org.apache.ratis.thirdparty.com.google.protobuf;
-
-/** Utilities for {@link ByteString}. */
-public interface ByteStringUtils {
-  /**
-   * Wrap the given array.
-   * Note that the content of the array must not be changed.
-   * Otherwise, it violates the immutability of {@link ByteString}.
-   */
-  static ByteString unsafeWrap(byte[] array) {
-    return ByteString.wrap(array);
-  }
-}
diff --git 
a/ratis-examples/src/main/java/org/apache/ratis/examples/arithmetic/ArithmeticStateMachine.java
 
b/ratis-examples/src/main/java/org/apache/ratis/examples/arithmetic/ArithmeticStateMachine.java
index ac7b199fc..28e3fb1c7 100644
--- 
a/ratis-examples/src/main/java/org/apache/ratis/examples/arithmetic/ArithmeticStateMachine.java
+++ 
b/ratis-examples/src/main/java/org/apache/ratis/examples/arithmetic/ArithmeticStateMachine.java
@@ -34,14 +34,13 @@ import org.apache.ratis.statemachine.impl.BaseStateMachine;
 import org.apache.ratis.statemachine.impl.SimpleStateMachineStorage;
 import org.apache.ratis.statemachine.impl.SingleFileSnapshotInfo;
 import org.apache.ratis.util.AutoCloseableLock;
+import org.apache.ratis.util.FileUtils;
 import org.apache.ratis.util.JavaUtils;
 import org.apache.ratis.util.MD5FileUtil;
 
 import java.io.BufferedInputStream;
 import java.io.BufferedOutputStream;
 import java.io.File;
-import java.io.FileInputStream;
-import java.io.FileOutputStream;
 import java.io.IOException;
 import java.io.ObjectInputStream;
 import java.io.ObjectOutputStream;
@@ -97,8 +96,8 @@ public class ArithmeticStateMachine extends BaseStateMachine {
     final File snapshotFile =  storage.getSnapshotFile(last.getTerm(), 
last.getIndex());
     LOG.info("Taking a snapshot to file {}", snapshotFile);
 
-    try(ObjectOutputStream out = new ObjectOutputStream(
-        new BufferedOutputStream(new FileOutputStream(snapshotFile)))) {
+    try(ObjectOutputStream out = new ObjectOutputStream(new 
BufferedOutputStream(
+        FileUtils.newOutputStream(snapshotFile)))) {
       out.writeObject(copy);
     } catch(IOException ioe) {
       LOG.warn("Failed to write snapshot file \"" + snapshotFile
@@ -130,8 +129,8 @@ public class ArithmeticStateMachine extends 
BaseStateMachine {
 
     final TermIndex last = 
SimpleStateMachineStorage.getTermIndexFromSnapshotFile(snapshotFile);
     try(AutoCloseableLock writeLock = writeLock();
-        ObjectInputStream in = new ObjectInputStream(
-            new BufferedInputStream(new FileInputStream(snapshotFile)))) {
+        ObjectInputStream in = new ObjectInputStream(new BufferedInputStream(
+            FileUtils.newInputStream(snapshotFile)))) {
       reset();
       setLastAppliedTermIndex(last);
       variables.putAll(JavaUtils.cast(in.readObject()));
diff --git 
a/ratis-examples/src/main/java/org/apache/ratis/examples/filestore/cli/DataStream.java
 
b/ratis-examples/src/main/java/org/apache/ratis/examples/filestore/cli/DataStream.java
index bb9942a80..4e4b4dbc6 100644
--- 
a/ratis-examples/src/main/java/org/apache/ratis/examples/filestore/cli/DataStream.java
+++ 
b/ratis-examples/src/main/java/org/apache/ratis/examples/filestore/cli/DataStream.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
@@ -26,14 +26,15 @@ import org.apache.ratis.protocol.DataStreamReply;
 import org.apache.ratis.protocol.RoutingTable;
 import org.apache.ratis.thirdparty.io.netty.buffer.ByteBuf;
 import org.apache.ratis.thirdparty.io.netty.buffer.PooledByteBufAllocator;
+import org.apache.ratis.util.FileUtils;
 import org.apache.ratis.util.JavaUtils;
 import org.apache.ratis.util.Preconditions;
 
 import java.io.File;
-import java.io.FileInputStream;
 import java.io.IOException;
 import java.nio.MappedByteBuffer;
 import java.nio.channels.FileChannel;
+import java.nio.file.StandardOpenOption;
 import java.util.ArrayList;
 import java.util.Arrays;
 import java.util.Collections;
@@ -238,8 +239,7 @@ public class DataStream extends Client {
 
       final List<CompletableFuture<DataStreamReply>> futures = new 
ArrayList<>();
       final DataStreamOutput out = client.getStreamOutput(file.getName(), 
fileSize, routingTable);
-      try (FileInputStream fis = new FileInputStream(file)) {
-        final FileChannel in = fis.getChannel();
+      try (FileChannel in = FileUtils.newFileChannel(file, 
StandardOpenOption.READ)) {
         for (long offset = 0L; offset < fileSize; ) {
           offset += write(in, out, offset, futures);
         }
diff --git 
a/ratis-examples/src/main/java/org/apache/ratis/examples/filestore/cli/LoadGen.java
 
b/ratis-examples/src/main/java/org/apache/ratis/examples/filestore/cli/LoadGen.java
index 3fd4714d2..8225df9ee 100644
--- 
a/ratis-examples/src/main/java/org/apache/ratis/examples/filestore/cli/LoadGen.java
+++ 
b/ratis-examples/src/main/java/org/apache/ratis/examples/filestore/cli/LoadGen.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
@@ -22,11 +22,12 @@ import com.beust.jcommander.Parameters;
 import org.apache.ratis.examples.filestore.FileStoreClient;
 import org.apache.ratis.thirdparty.io.netty.buffer.ByteBuf;
 import org.apache.ratis.thirdparty.io.netty.buffer.PooledByteBufAllocator;
+import org.apache.ratis.util.FileUtils;
 
 import java.io.File;
-import java.io.FileInputStream;
 import java.io.IOException;
 import java.nio.channels.FileChannel;
+import java.nio.file.StandardOpenOption;
 import java.util.ArrayList;
 import java.util.HashMap;
 import java.util.List;
@@ -97,8 +98,7 @@ public class LoadGen extends Client {
       CompletableFuture.supplyAsync(() -> {
         List<CompletableFuture<Long>> futures = new ArrayList<>();
         File file = new File(path);
-        try (FileInputStream fis = new FileInputStream(file)) {
-          final FileChannel in = fis.getChannel();
+        try (FileChannel in = FileUtils.newFileChannel(file, 
StandardOpenOption.READ)) {
           for (long offset = 0L; offset < getFileSizeInBytes(); ) {
             offset += write(in, offset, client, file.getName(), futures);
           }
diff --git 
a/ratis-server/src/main/java/org/apache/ratis/server/raftlog/segmented/SegmentedRaftLogReader.java
 
b/ratis-server/src/main/java/org/apache/ratis/server/raftlog/segmented/SegmentedRaftLogReader.java
index 53cb06186..c8170f9b1 100644
--- 
a/ratis-server/src/main/java/org/apache/ratis/server/raftlog/segmented/SegmentedRaftLogReader.java
+++ 
b/ratis-server/src/main/java/org/apache/ratis/server/raftlog/segmented/SegmentedRaftLogReader.java
@@ -19,12 +19,13 @@ package org.apache.ratis.server.raftlog.segmented;
 
 import org.apache.ratis.io.CorruptedFileException;
 import org.apache.ratis.metrics.Timekeeper;
+import org.apache.ratis.proto.RaftProtos.LogEntryProto;
 import org.apache.ratis.protocol.exceptions.ChecksumException;
 import org.apache.ratis.server.metrics.SegmentedRaftLogMetrics;
 import org.apache.ratis.server.raftlog.RaftLog;
 import org.apache.ratis.thirdparty.com.google.protobuf.CodedInputStream;
 import org.apache.ratis.thirdparty.com.google.protobuf.CodedOutputStream;
-import org.apache.ratis.proto.RaftProtos.LogEntryProto;
+import org.apache.ratis.util.FileUtils;
 import org.apache.ratis.util.IOUtils;
 import org.apache.ratis.util.Preconditions;
 import org.apache.ratis.util.PureJavaCrc32C;
@@ -33,7 +34,14 @@ import org.apache.ratis.util.StringUtils;
 import org.slf4j.Logger;
 import org.slf4j.LoggerFactory;
 
-import java.io.*;
+import java.io.BufferedInputStream;
+import java.io.Closeable;
+import java.io.DataInputStream;
+import java.io.EOFException;
+import java.io.File;
+import java.io.FilterInputStream;
+import java.io.IOException;
+import java.io.InputStream;
 import java.util.Optional;
 import java.util.zip.Checksum;
 
@@ -41,7 +49,7 @@ class SegmentedRaftLogReader implements Closeable {
   static final Logger LOG = 
LoggerFactory.getLogger(SegmentedRaftLogReader.class);
   /**
    * InputStream wrapper that keeps track of the current stream position.
-   *
+   * <p>
    * This stream also allows us to set a limit on how many bytes we can read
    * without getting an exception.
    */
@@ -141,11 +149,9 @@ class SegmentedRaftLogReader implements Closeable {
   private final SegmentedRaftLogMetrics raftLogMetrics;
   private final SizeInBytes maxOpSize;
 
-  SegmentedRaftLogReader(File file, SizeInBytes maxOpSize, 
SegmentedRaftLogMetrics raftLogMetrics)
-      throws FileNotFoundException {
+  SegmentedRaftLogReader(File file, SizeInBytes maxOpSize, 
SegmentedRaftLogMetrics raftLogMetrics) throws IOException {
     this.file = file;
-    this.limiter = new LimitedInputStream(
-        new BufferedInputStream(new FileInputStream(file)));
+    this.limiter = new LimitedInputStream(new 
BufferedInputStream(FileUtils.newInputStream(file)));
     in = new DataInputStream(limiter);
     checksum = new PureJavaCrc32C();
     this.maxOpSize = maxOpSize;
@@ -154,13 +160,10 @@ class SegmentedRaftLogReader implements Closeable {
 
   /**
    * Read header from the log file:
-   *
    * (1) The header in file is verified successfully.
    *     Then, return true.
-   *
    * (2) The header in file is partially written.
    *     Then, return false.
-   *
    * (3) The header in file is corrupted or there is some other {@link 
IOException}.
    *     Then, throw an exception.
    */
@@ -197,7 +200,7 @@ class SegmentedRaftLogReader implements Closeable {
     final Timekeeper timekeeper = Optional.ofNullable(raftLogMetrics)
         .map(SegmentedRaftLogMetrics::getReadEntryTimer)
         .orElse(null);
-    try(AutoCloseable readEntryContext = Timekeeper.start(timekeeper)) {
+    try(AutoCloseable ignored = Timekeeper.start(timekeeper)) {
       return decodeEntry();
     } catch (EOFException eof) {
       in.reset();
diff --git 
a/ratis-server/src/main/java/org/apache/ratis/server/storage/FileChunkReader.java
 
b/ratis-server/src/main/java/org/apache/ratis/server/storage/FileChunkReader.java
index dfed2790f..65bfc8b80 100644
--- 
a/ratis-server/src/main/java/org/apache/ratis/server/storage/FileChunkReader.java
+++ 
b/ratis-server/src/main/java/org/apache/ratis/server/storage/FileChunkReader.java
@@ -21,12 +21,12 @@ import org.apache.ratis.io.MD5Hash;
 import org.apache.ratis.proto.RaftProtos.FileChunkProto;
 import org.apache.ratis.thirdparty.com.google.protobuf.ByteString;
 import org.apache.ratis.thirdparty.com.google.protobuf.UnsafeByteOperations;
+import org.apache.ratis.util.FileUtils;
 import org.apache.ratis.util.IOUtils;
 import org.apache.ratis.util.JavaUtils;
 
 import java.io.Closeable;
 import java.io.File;
-import java.io.FileInputStream;
 import java.io.IOException;
 import java.io.InputStream;
 import java.nio.file.Path;
@@ -57,13 +57,19 @@ public class FileChunkReader implements Closeable {
     final File f = info.getPath().toFile();
     if (info.getFileDigest() == null) {
       digester = MD5Hash.getDigester();
-      this.in = new DigestInputStream(new FileInputStream(f), digester);
+      this.in = new DigestInputStream(FileUtils.newInputStream(f), digester);
     } else {
       digester = null;
-      this.in = new FileInputStream(f);
+      this.in = FileUtils.newInputStream(f);
     }
   }
 
+  static ByteString readFileChunk(int chunkLength, InputStream in) throws 
IOException {
+    final byte[] chunkBuffer = new byte[chunkLength];
+    IOUtils.readFully(in, chunkBuffer, 0, chunkBuffer.length);
+    return UnsafeByteOperations.unsafeWrap(chunkBuffer);
+  }
+
   /**
    * Read the next chunk.
    *
@@ -74,9 +80,7 @@ public class FileChunkReader implements Closeable {
   public FileChunkProto readFileChunk(int chunkMaxSize) throws IOException {
     final long remaining = info.getFileSize() - offset;
     final int chunkLength = remaining < chunkMaxSize ? (int) remaining : 
chunkMaxSize;
-    final byte[] chunkBuffer = new byte[chunkLength];
-    IOUtils.readFully(in, chunkBuffer, 0, chunkBuffer.length);
-    final ByteString data = UnsafeByteOperations.unsafeWrap(chunkBuffer);
+    final ByteString data = readFileChunk(chunkLength, in);
     // whether this chunk is the last chunk of current file
     final boolean isDone = offset + chunkLength == info.getFileSize();
     final ByteString fileDigest;
diff --git 
a/ratis-server/src/main/java/org/apache/ratis/server/storage/RaftStorageImpl.java
 
b/ratis-server/src/main/java/org/apache/ratis/server/storage/RaftStorageImpl.java
index 4fcf46389..fbb7bf7d4 100644
--- 
a/ratis-server/src/main/java/org/apache/ratis/server/storage/RaftStorageImpl.java
+++ 
b/ratis-server/src/main/java/org/apache/ratis/server/storage/RaftStorageImpl.java
@@ -17,21 +17,23 @@
  */
 package org.apache.ratis.server.storage;
 
+import java.io.InputStream;
+import java.io.OutputStream;
+import java.nio.file.NoSuchFileException;
 import java.util.concurrent.atomic.AtomicReference;
 import org.apache.ratis.proto.RaftProtos.LogEntryProto;
 import org.apache.ratis.server.RaftConfiguration;
 import org.apache.ratis.server.RaftServerConfigKeys.Log.CorruptionPolicy;
 import org.apache.ratis.server.raftlog.LogProtoUtils;
 import org.apache.ratis.server.storage.RaftStorageDirectoryImpl.StorageState;
+import org.apache.ratis.util.AtomicFileOutputStream;
+import org.apache.ratis.util.FileUtils;
 import org.apache.ratis.util.JavaUtils;
 import org.apache.ratis.util.SizeInBytes;
 
 import java.io.File;
 import java.io.FileNotFoundException;
 import java.io.IOException;
-import java.io.FileInputStream;
-import java.io.FileOutputStream;
-import java.nio.file.Files;
 import java.util.Optional;
 
 /** The storage of a {@link org.apache.ratis.server.RaftServer}. */
@@ -98,7 +100,7 @@ public class RaftStorageImpl implements RaftStorage {
   }
 
   private void cleanMetaTmpFile() throws IOException {
-    Files.deleteIfExists(storageDir.getMetaTmpFile().toPath());
+    FileUtils.deleteIfExists(storageDir.getMetaTmpFile());
   }
 
   private StorageState analyzeAndRecoverStorage(boolean toLock) throws 
IOException {
@@ -142,7 +144,7 @@ public class RaftStorageImpl implements RaftStorage {
 
   public void writeRaftConfiguration(LogEntryProto conf) {
     File confFile = storageDir.getMetaConfFile();
-    try (FileOutputStream fio = new FileOutputStream(confFile)) {
+    try (OutputStream fio = new AtomicFileOutputStream(confFile)) {
       conf.writeTo(fio);
     } catch (Exception e) {
       LOG.error("Failed writing configuration to file:" + confFile, e);
@@ -151,10 +153,10 @@ public class RaftStorageImpl implements RaftStorage {
 
   public RaftConfiguration readRaftConfiguration() {
     File confFile = storageDir.getMetaConfFile();
-    try (FileInputStream fio = new FileInputStream(confFile)) {
+    try (InputStream fio = FileUtils.newInputStream(confFile)) {
       LogEntryProto confProto = 
LogEntryProto.newBuilder().mergeFrom(fio).build();
       return LogProtoUtils.toRaftConfiguration(confProto);
-    } catch (FileNotFoundException e) {
+    } catch (FileNotFoundException | NoSuchFileException e) {
       return null;
     } catch (Exception e) {
       LOG.error("Failed reading configuration from file:" + confFile, e);
diff --git 
a/ratis-server/src/main/java/org/apache/ratis/server/storage/RaftStorageMetadataFileImpl.java
 
b/ratis-server/src/main/java/org/apache/ratis/server/storage/RaftStorageMetadataFileImpl.java
index 1fb723f81..896a5f21e 100644
--- 
a/ratis-server/src/main/java/org/apache/ratis/server/storage/RaftStorageMetadataFileImpl.java
+++ 
b/ratis-server/src/main/java/org/apache/ratis/server/storage/RaftStorageMetadataFileImpl.java
@@ -20,12 +20,12 @@ package org.apache.ratis.server.storage;
 import org.apache.ratis.protocol.RaftPeerId;
 import org.apache.ratis.util.AtomicFileOutputStream;
 import org.apache.ratis.util.ConcurrentUtils;
+import org.apache.ratis.util.FileUtils;
 import org.apache.ratis.util.JavaUtils;
 
 import java.io.BufferedReader;
 import java.io.BufferedWriter;
 import java.io.File;
-import java.io.FileInputStream;
 import java.io.IOException;
 import java.io.InputStreamReader;
 import java.io.OutputStreamWriter;
@@ -110,7 +110,7 @@ class RaftStorageMetadataFileImpl implements 
RaftStorageMetadataFile {
       return RaftStorageMetadata.getDefault();
     }
     try(BufferedReader br = new BufferedReader(new InputStreamReader(
-        new FileInputStream(file), StandardCharsets.UTF_8))) {
+        FileUtils.newInputStream(file), StandardCharsets.UTF_8))) {
       Properties properties = new Properties();
       properties.load(br);
       return RaftStorageMetadata.valueOf(getTerm(properties), 
getVotedFor(properties));
diff --git 
a/ratis-server/src/main/java/org/apache/ratis/server/storage/SnapshotManager.java
 
b/ratis-server/src/main/java/org/apache/ratis/server/storage/SnapshotManager.java
index 8d291acb2..c49a86ec5 100644
--- 
a/ratis-server/src/main/java/org/apache/ratis/server/storage/SnapshotManager.java
+++ 
b/ratis-server/src/main/java/org/apache/ratis/server/storage/SnapshotManager.java
@@ -27,7 +27,6 @@ import org.apache.ratis.statemachine.SnapshotInfo;
 import org.apache.ratis.statemachine.StateMachine;
 import org.apache.ratis.statemachine.StateMachineStorage;
 import org.apache.ratis.util.FileUtils;
-import org.apache.ratis.util.IOUtils;
 import org.apache.ratis.util.JavaUtils;
 import org.apache.ratis.util.MD5FileUtil;
 import org.apache.ratis.util.MemoizedSupplier;
@@ -37,11 +36,12 @@ import org.slf4j.Logger;
 import org.slf4j.LoggerFactory;
 
 import java.io.File;
-import java.io.FileOutputStream;
+import java.io.FileNotFoundException;
 import java.io.IOException;
+import java.nio.ByteBuffer;
 import java.nio.channels.FileChannel;
 import java.nio.file.Path;
-import java.security.DigestOutputStream;
+import java.nio.file.StandardOpenOption;
 import java.security.MessageDigest;
 import java.util.Optional;
 import java.util.function.Function;
@@ -77,6 +77,28 @@ public class SnapshotManager {
         new File(dir.get().getRoot(), c.getFilename()).toPath()).toString();
   }
 
+  private FileChannel open(FileChunkProto chunk, File tmpSnapshotFile) throws 
IOException {
+    final FileChannel out;
+    final boolean exists = tmpSnapshotFile.exists();
+    if (chunk.getOffset() == 0) {
+      // if offset is 0, delete any existing temp snapshot file if it has the 
same last index.
+      if (exists) {
+        FileUtils.deleteFully(tmpSnapshotFile);
+      }
+      // create the temp snapshot file and put padding inside
+      out = FileUtils.newFileChannel(tmpSnapshotFile, 
StandardOpenOption.WRITE, StandardOpenOption.CREATE);
+      digester.get().reset();
+    } else {
+      if (!exists) {
+        throw new FileNotFoundException("Chunk offset is non-zero but file is 
not found: " + tmpSnapshotFile
+            + ", chunk=" + chunk);
+      }
+      out = FileUtils.newFileChannel(tmpSnapshotFile, StandardOpenOption.WRITE)
+          .position(chunk.getOffset());
+    }
+    return out;
+  }
+
   public void installSnapshot(InstallSnapshotRequestProto request, 
StateMachine stateMachine) throws IOException {
     final InstallSnapshotRequestProto.SnapshotChunkProto snapshotChunkRequest 
= request.getSnapshotChunk();
     final long lastIncludedIndex = 
snapshotChunkRequest.getTermIndex().getIndex();
@@ -103,30 +125,15 @@ public class SnapshotManager {
       final File tmpSnapshotFile = new File(tmpDir, 
getRelativePath.apply(chunk));
       
FileUtils.createDirectoriesDeleteExistingNonDirectory(tmpSnapshotFile.getParentFile());
 
-      FileOutputStream out = null;
-      try {
-        // if offset is 0, delete any existing temp snapshot file if it has the
-        // same last index.
-        if (chunk.getOffset() == 0) {
-          if (tmpSnapshotFile.exists()) {
-            FileUtils.deleteFully(tmpSnapshotFile);
-          }
-          // create the temp snapshot file and put padding inside
-          out = new FileOutputStream(tmpSnapshotFile);
-          digester.get().reset();
-        } else {
-          Preconditions.assertTrue(tmpSnapshotFile.exists());
-          out = new FileOutputStream(tmpSnapshotFile, true);
-          FileChannel fc = out.getChannel();
-          fc.position(chunk.getOffset());
-        }
+      try (FileChannel out = open(chunk, tmpSnapshotFile)) {
+        final ByteBuffer data = chunk.getData().asReadOnlyByteBuffer();
+        digester.get().update(data.duplicate());
 
-        // write data to the file
-        try (DigestOutputStream digestOut = new DigestOutputStream(out, 
digester.get())) {
-          digestOut.write(chunk.getData().toByteArray());
+        int written = 0;
+        for(; data.remaining() > 0; ) {
+          written += out.write(data);
         }
-      } finally {
-        IOUtils.cleanup(null, out);
+        Preconditions.assertSame(chunk.getData().size(), written, "written");
       }
 
       // rename the temp snapshot file if this is the last chunk. also verify
diff --git 
a/ratis-test/src/test/java/org/apache/ratis/datastream/DataStreamClusterTests.java
 
b/ratis-test/src/test/java/org/apache/ratis/datastream/DataStreamClusterTests.java
index dcb54be3d..959cb3fc0 100644
--- 
a/ratis-test/src/test/java/org/apache/ratis/datastream/DataStreamClusterTests.java
+++ 
b/ratis-test/src/test/java/org/apache/ratis/datastream/DataStreamClusterTests.java
@@ -32,13 +32,15 @@ import org.apache.ratis.protocol.RaftClientReply;
 import org.apache.ratis.protocol.RaftClientRequest;
 import org.apache.ratis.server.RaftServer;
 import org.apache.ratis.util.CollectionUtils;
+import org.apache.ratis.util.FileUtils;
 import org.apache.ratis.util.Timestamp;
 import org.apache.ratis.util.function.CheckedConsumer;
 import org.junit.Assert;
 import org.junit.Test;
 
 import java.io.File;
-import java.io.FileInputStream;
+import java.nio.channels.FileChannel;
+import java.nio.file.StandardOpenOption;
 import java.util.Collection;
 import java.util.concurrent.CompletableFuture;
 import java.util.concurrent.ThreadLocalRandom;
@@ -151,8 +153,8 @@ public abstract class DataStreamClusterTests<CLUSTER 
extends MiniRaftCluster> ex
     return new CheckedConsumer<DataStreamOutputImpl, Exception>() {
       @Override
       public void accept(DataStreamOutputImpl out) throws Exception {
-        try (FileInputStream in = new FileInputStream(f)) {
-          final long transferred = in.getChannel().transferTo(0, size, 
out.getWritableByteChannel());
+        try (FileChannel in = FileUtils.newFileChannel(f, 
StandardOpenOption.READ)) {
+          final long transferred = in.transferTo(0, size, 
out.getWritableByteChannel());
           Assert.assertEquals(size, transferred);
         }
       }
diff --git 
a/ratis-test/src/test/java/org/apache/ratis/datastream/DataStreamTestUtils.java 
b/ratis-test/src/test/java/org/apache/ratis/datastream/DataStreamTestUtils.java
index 906071dfa..7666bacd9 100644
--- 
a/ratis-test/src/test/java/org/apache/ratis/datastream/DataStreamTestUtils.java
+++ 
b/ratis-test/src/test/java/org/apache/ratis/datastream/DataStreamTestUtils.java
@@ -54,9 +54,10 @@ import org.slf4j.Logger;
 import org.slf4j.LoggerFactory;
 
 import java.io.File;
-import java.io.FileOutputStream;
 import java.nio.ByteBuffer;
+import java.nio.channels.FileChannel;
 import java.nio.channels.ReadableByteChannel;
+import java.nio.file.StandardOpenOption;
 import java.util.ArrayList;
 import java.util.Collection;
 import java.util.List;
@@ -114,8 +115,8 @@ public interface DataStreamTestUtils {
       }
     };
     FileUtils.createDirectories(f.getParentFile());
-    try(FileOutputStream out = new FileOutputStream(f)) {
-      final long transferred = out.getChannel().transferFrom(source, 0, size);
+    try(FileChannel out = FileUtils.newFileChannel(f, 
StandardOpenOption.CREATE, StandardOpenOption.WRITE)) {
+      final long transferred = out.transferFrom(source, 0, size);
       Assert.assertEquals(size, transferred);
     }
   }
diff --git 
a/ratis-test/src/test/java/org/apache/ratis/security/SecurityTestUtils.java 
b/ratis-test/src/test/java/org/apache/ratis/security/SecurityTestUtils.java
index a981282fd..d6222b227 100644
--- a/ratis-test/src/test/java/org/apache/ratis/security/SecurityTestUtils.java
+++ b/ratis-test/src/test/java/org/apache/ratis/security/SecurityTestUtils.java
@@ -20,6 +20,7 @@ package org.apache.ratis.security;
 import org.apache.ratis.security.TlsConf.Builder;
 import org.apache.ratis.security.TlsConf.CertificatesConf;
 import org.apache.ratis.security.TlsConf.PrivateKeyConf;
+import org.apache.ratis.util.FileUtils;
 import org.bouncycastle.util.io.pem.PemObject;
 import org.bouncycastle.util.io.pem.PemReader;
 import org.junit.Assert;
@@ -32,8 +33,8 @@ import javax.net.ssl.TrustManager;
 import javax.net.ssl.TrustManagerFactory;
 import javax.net.ssl.X509TrustManager;
 import java.io.File;
-import java.io.FileInputStream;
 import java.io.FileReader;
+import java.io.InputStream;
 import java.net.URL;
 import java.security.KeyFactory;
 import java.security.KeyStore;
@@ -110,7 +111,7 @@ public interface SecurityTestUtils {
       // Read certificates
       X509Certificate[] certificate = new X509Certificate[1];
       CertificateFactory fact = CertificateFactory.getInstance("X.509");
-      try (FileInputStream is = new FileInputStream(getResource(certPath))) {
+      try (InputStream is = FileUtils.newInputStream(getResource(certPath))) {
         certificate[0] = (X509Certificate) fact.generateCertificate(is);
       }
       return certificate;
diff --git 
a/ratis-test/src/test/java/org/apache/ratis/server/raftlog/segmented/TestLogSegment.java
 
b/ratis-test/src/test/java/org/apache/ratis/server/raftlog/segmented/TestLogSegment.java
index 0a58f44f6..755476bf5 100644
--- 
a/ratis-test/src/test/java/org/apache/ratis/server/raftlog/segmented/TestLogSegment.java
+++ 
b/ratis-test/src/test/java/org/apache/ratis/server/raftlog/segmented/TestLogSegment.java
@@ -108,9 +108,11 @@ public class TestLogSegment extends BaseTest {
       // 0 < truncatedEntrySize < entrySize
       final long fileLength = file.length();
       final long truncatedFileLength = fileLength - (entrySize - 
truncatedEntrySize);
+      Assert.assertTrue(truncatedFileLength < fileLength);
       LOG.info("truncate last entry: entry(size={}, truncated={}), 
file(length={}, truncated={})",
           entrySize, truncatedEntrySize, fileLength, truncatedFileLength);
       FileUtils.truncateFile(file, truncatedFileLength);
+      Assert.assertEquals(truncatedFileLength, file.length());
     }
 
     storage.close();
diff --git 
a/ratis-test/src/test/java/org/apache/ratis/server/raftlog/segmented/TestRaftLogReadWrite.java
 
b/ratis-test/src/test/java/org/apache/ratis/server/raftlog/segmented/TestRaftLogReadWrite.java
index e79f9f7f9..a020b43bd 100644
--- 
a/ratis-test/src/test/java/org/apache/ratis/server/raftlog/segmented/TestRaftLogReadWrite.java
+++ 
b/ratis-test/src/test/java/org/apache/ratis/server/raftlog/segmented/TestRaftLogReadWrite.java
@@ -35,10 +35,11 @@ import org.junit.Before;
 import org.junit.Test;
 
 import java.io.File;
-import java.io.FileOutputStream;
 import java.io.IOException;
 import java.io.RandomAccessFile;
 import java.nio.ByteBuffer;
+import java.nio.channels.FileChannel;
+import java.nio.file.StandardOpenOption;
 import java.util.ArrayList;
 import java.util.Collections;
 import java.util.List;
@@ -201,10 +202,10 @@ public class TestRaftLogReadWrite extends BaseTest {
     // make sure the file contains padding
     Assert.assertEquals(4 * 1024 * 1024, openSegment.length());
 
-    try (FileOutputStream fout = new FileOutputStream(openSegment, true)) {
-      ByteBuffer byteBuffer = ByteBuffer.wrap(new byte[]{-1, 1});
-      fout.getChannel()
-          .write(byteBuffer, 16 * 1024 * 1024 - 10);
+    try (FileChannel fout = FileUtils.newFileChannel(openSegment, 
StandardOpenOption.WRITE)) {
+      final byte[] array = {-1, 1};
+      final int written = fout.write(ByteBuffer.wrap(array), 16 * 1024 * 1024 
- 10);
+      Assert.assertEquals(array.length, written);
     }
 
     List<LogEntryProto> list = new ArrayList<>();

Reply via email to