Repository: cassandra Updated Branches: refs/heads/trunk f3dfb1503 -> 74bdf633e
remove use of FileInputStream/FileOutputStream for gc concerns patcb by dbrosius reviewed by rstupp for CASSANDRA-13452 Project: http://git-wip-us.apache.org/repos/asf/cassandra/repo Commit: http://git-wip-us.apache.org/repos/asf/cassandra/commit/74bdf633 Tree: http://git-wip-us.apache.org/repos/asf/cassandra/tree/74bdf633 Diff: http://git-wip-us.apache.org/repos/asf/cassandra/diff/74bdf633 Branch: refs/heads/trunk Commit: 74bdf633e9d6e03237b9b5285ce23b39d6846c3f Parents: f3dfb15 Author: Dave Brosius <dbros...@mebigfatguy.com> Authored: Mon Apr 17 07:33:48 2017 -0400 Committer: Dave Brosius <dbros...@mebigfatguy.com> Committed: Mon Apr 17 07:33:48 2017 -0400 ---------------------------------------------------------------------- .../cassandra/db/commitlog/CommitLogReader.java | 8 ++-- .../apache/cassandra/gms/FailureDetector.java | 16 +++++--- .../cassandra/hadoop/cql3/CqlConfigHelper.java | 40 ++++++++++---------- .../io/compress/CompressionMetadata.java | 4 +- .../io/sstable/format/SSTableReader.java | 6 ++- .../cassandra/security/JKSKeyProvider.java | 6 ++- .../apache/cassandra/security/SSLFactory.java | 13 ++++--- .../cassandra/tools/SSTableMetadataViewer.java | 3 +- .../apache/cassandra/stress/StressGraph.java | 4 +- .../cassandra/stress/settings/SettingsNode.java | 12 +++--- 10 files changed, 66 insertions(+), 46 deletions(-) ---------------------------------------------------------------------- http://git-wip-us.apache.org/repos/asf/cassandra/blob/74bdf633/src/java/org/apache/cassandra/db/commitlog/CommitLogReader.java ---------------------------------------------------------------------- diff --git a/src/java/org/apache/cassandra/db/commitlog/CommitLogReader.java b/src/java/org/apache/cassandra/db/commitlog/CommitLogReader.java index 9eec477..5fb1d5d 100644 --- a/src/java/org/apache/cassandra/db/commitlog/CommitLogReader.java +++ b/src/java/org/apache/cassandra/db/commitlog/CommitLogReader.java @@ -17,6 +17,8 @@ */ package org.apache.cassandra.db.commitlog; +import java.nio.file.Files; +import java.nio.file.Path; import java.io.*; import java.util.*; import java.util.concurrent.atomic.AtomicInteger; @@ -397,9 +399,9 @@ public class CommitLogReader catch (Throwable t) { JVMStabilityInspector.inspectThrowable(t); - File f = File.createTempFile("mutation", "dat"); + Path p = Files.createTempFile("mutation", "dat"); - try (DataOutputStream out = new DataOutputStream(new FileOutputStream(f))) + try (DataOutputStream out = new DataOutputStream(Files.newOutputStream(p))) { out.write(inputBuffer, 0, size); } @@ -409,7 +411,7 @@ public class CommitLogReader String.format( "Unexpected error deserializing mutation; saved to %s. " + "This may be caused by replaying a mutation against a table with the same name but incompatible schema. " + - "Exception follows: %s", f.getAbsolutePath(), t), + "Exception follows: %s", p.toString(), t), CommitLogReadErrorReason.MUTATION_ERROR, false)); return; http://git-wip-us.apache.org/repos/asf/cassandra/blob/74bdf633/src/java/org/apache/cassandra/gms/FailureDetector.java ---------------------------------------------------------------------- diff --git a/src/java/org/apache/cassandra/gms/FailureDetector.java b/src/java/org/apache/cassandra/gms/FailureDetector.java index e5a29d6..8aad472 100644 --- a/src/java/org/apache/cassandra/gms/FailureDetector.java +++ b/src/java/org/apache/cassandra/gms/FailureDetector.java @@ -17,6 +17,9 @@ */ package org.apache.cassandra.gms; +import java.nio.file.Files; +import java.nio.file.StandardOpenOption; +import java.nio.file.Path; import java.io.*; import java.lang.management.ManagementFactory; import java.net.InetAddress; @@ -215,15 +218,18 @@ public class FailureDetector implements IFailureDetector, FailureDetectorMBean */ public void dumpInterArrivalTimes() { - File file = FileUtils.createTempFile("failuredetector-", ".dat"); + Path path = null; + try { + path = Files.createTempFile("failuredetector-", ".dat"); - try (OutputStream os = new BufferedOutputStream(new FileOutputStream(file, true))) - { - os.write(toString().getBytes()); + try (OutputStream os = new BufferedOutputStream(Files.newOutputStream(path, StandardOpenOption.APPEND))) + { + os.write(toString().getBytes()); + } } catch (IOException e) { - throw new FSWriteError(e, file); + throw new FSWriteError(e, (path == null) ? null : path.toFile()); } } http://git-wip-us.apache.org/repos/asf/cassandra/blob/74bdf633/src/java/org/apache/cassandra/hadoop/cql3/CqlConfigHelper.java ---------------------------------------------------------------------- diff --git a/src/java/org/apache/cassandra/hadoop/cql3/CqlConfigHelper.java b/src/java/org/apache/cassandra/hadoop/cql3/CqlConfigHelper.java index 4c71273..f9a6f3a 100644 --- a/src/java/org/apache/cassandra/hadoop/cql3/CqlConfigHelper.java +++ b/src/java/org/apache/cassandra/hadoop/cql3/CqlConfigHelper.java @@ -19,7 +19,9 @@ package org.apache.cassandra.hadoop.cql3; * under the License. * */ -import java.io.FileInputStream; +import java.nio.file.Files; +import java.nio.file.Paths; +import java.io.InputStream; import java.io.IOException; import java.security.KeyManagementException; import java.security.KeyStore; @@ -86,7 +88,7 @@ public class CqlConfigHelper private static final String OUTPUT_CQL = "cassandra.output.cql"; private static final String OUTPUT_NATIVE_PORT = "cassandra.output.native.port"; - + /** * Set the CQL columns for the input of this job. * @@ -97,10 +99,10 @@ public class CqlConfigHelper { if (columns == null || columns.isEmpty()) return; - + conf.set(INPUT_CQL_COLUMNS_CONFIG, columns); } - + /** * Set the CQL query Limit for the input of this job. * @@ -127,10 +129,10 @@ public class CqlConfigHelper { if (clauses == null || clauses.isEmpty()) return; - + conf.set(INPUT_CQL_WHERE_CLAUSE_CONFIG, clauses); } - + /** * Set the CQL prepared statement for the output of this job. * @@ -141,7 +143,7 @@ public class CqlConfigHelper { if (cql == null || cql.isEmpty()) return; - + conf.set(OUTPUT_CQL, cql); } @@ -283,7 +285,7 @@ public class CqlConfigHelper return conf.get(OUTPUT_CQL); } - private static Optional<Integer> getProtocolVersion(Configuration conf) + private static Optional<Integer> getProtocolVersion(Configuration conf) { return getIntSetting(INPUT_NATIVE_PROTOCOL_VERSION, conf); } @@ -331,7 +333,7 @@ public class CqlConfigHelper if (sslOptions.isPresent()) builder.withSSL(sslOptions.get()); - if (protocolVersion.isPresent()) + if (protocolVersion.isPresent()) { builder.withProtocolVersion(ProtocolVersion.fromInt(protocolVersion.get())); } @@ -356,7 +358,7 @@ public class CqlConfigHelper public static void setInputMaxSimultReqPerConnections(Configuration conf, String reqs) { conf.set(INPUT_NATIVE_MAX_SIMULT_REQ_PER_CONNECTION, reqs); - } + } public static void setInputNativeConnectionTimeout(Configuration conf, String timeout) { @@ -396,7 +398,7 @@ public class CqlConfigHelper public static void setInputNativeSSLTruststorePath(Configuration conf, String path) { conf.set(INPUT_NATIVE_SSL_TRUST_STORE_PATH, path); - } + } public static void setInputNativeSSLKeystorePath(Configuration conf, String path) { @@ -452,7 +454,7 @@ public class CqlConfigHelper } return poolingOptions; - } + } private static QueryOptions getReadQueryOptions(Configuration conf) { @@ -476,7 +478,7 @@ public class CqlConfigHelper Optional<Integer> sendBufferSize = getInputNativeSendBufferSize(conf); Optional<Integer> soLinger = getInputNativeSolinger(conf); Optional<Boolean> tcpNoDelay = getInputNativeTcpNodelay(conf); - Optional<Boolean> reuseAddress = getInputNativeReuseAddress(conf); + Optional<Boolean> reuseAddress = getInputNativeReuseAddress(conf); Optional<Boolean> keepAlive = getInputNativeKeepAlive(conf); if (connectTimeoutMillis.isPresent()) @@ -494,7 +496,7 @@ public class CqlConfigHelper if (reuseAddress.isPresent()) socketOptions.setReuseAddress(reuseAddress.get()); if (keepAlive.isPresent()) - socketOptions.setKeepAlive(keepAlive.get()); + socketOptions.setKeepAlive(keepAlive.get()); return socketOptions; } @@ -565,7 +567,7 @@ public class CqlConfigHelper String setting = conf.get(parameter); if (setting == null) return Optional.absent(); - return Optional.of(Integer.valueOf(setting)); + return Optional.of(Integer.valueOf(setting)); } private static Optional<Boolean> getBooleanSetting(String parameter, Configuration conf) @@ -573,7 +575,7 @@ public class CqlConfigHelper String setting = conf.get(parameter); if (setting == null) return Optional.absent(); - return Optional.of(Boolean.valueOf(setting)); + return Optional.of(Boolean.valueOf(setting)); } private static Optional<String> getStringSetting(String parameter, Configuration conf) @@ -581,7 +583,7 @@ public class CqlConfigHelper String setting = conf.get(parameter); if (setting == null) return Optional.absent(); - return Optional.of(setting); + return Optional.of(setting); } private static AuthProvider getClientAuthProvider(String factoryClassName, Configuration conf) @@ -623,7 +625,7 @@ public class CqlConfigHelper TrustManagerFactory tmf = null; if (truststorePath.isPresent()) { - try (FileInputStream tsf = new FileInputStream(truststorePath.get())) + try (InputStream tsf = Files.newInputStream(Paths.get(truststorePath.get()))) { KeyStore ts = KeyStore.getInstance("JKS"); ts.load(tsf, truststorePassword.isPresent() ? truststorePassword.get().toCharArray() : null); @@ -635,7 +637,7 @@ public class CqlConfigHelper KeyManagerFactory kmf = null; if (keystorePath.isPresent()) { - try (FileInputStream ksf = new FileInputStream(keystorePath.get())) + try (InputStream ksf = Files.newInputStream(Paths.get(keystorePath.get()))) { KeyStore ks = KeyStore.getInstance("JKS"); ks.load(ksf, keystorePassword.isPresent() ? keystorePassword.get().toCharArray() : null); http://git-wip-us.apache.org/repos/asf/cassandra/blob/74bdf633/src/java/org/apache/cassandra/io/compress/CompressionMetadata.java ---------------------------------------------------------------------- diff --git a/src/java/org/apache/cassandra/io/compress/CompressionMetadata.java b/src/java/org/apache/cassandra/io/compress/CompressionMetadata.java index a591bb5..6c1849f 100644 --- a/src/java/org/apache/cassandra/io/compress/CompressionMetadata.java +++ b/src/java/org/apache/cassandra/io/compress/CompressionMetadata.java @@ -17,6 +17,8 @@ */ package org.apache.cassandra.io.compress; +import java.nio.file.Files; +import java.nio.file.Paths; import java.io.BufferedOutputStream; import java.io.DataInput; import java.io.DataInputStream; @@ -103,7 +105,7 @@ public class CompressionMetadata { this.indexFilePath = indexFilePath; - try (DataInputStream stream = new DataInputStream(new FileInputStream(indexFilePath))) + try (DataInputStream stream = new DataInputStream(Files.newInputStream(Paths.get(indexFilePath)))) { String compressorName = stream.readUTF(); int optionCount = stream.readInt(); http://git-wip-us.apache.org/repos/asf/cassandra/blob/74bdf633/src/java/org/apache/cassandra/io/sstable/format/SSTableReader.java ---------------------------------------------------------------------- diff --git a/src/java/org/apache/cassandra/io/sstable/format/SSTableReader.java b/src/java/org/apache/cassandra/io/sstable/format/SSTableReader.java index 2a6e405..568336e 100644 --- a/src/java/org/apache/cassandra/io/sstable/format/SSTableReader.java +++ b/src/java/org/apache/cassandra/io/sstable/format/SSTableReader.java @@ -17,6 +17,8 @@ */ package org.apache.cassandra.io.sstable.format; +import java.nio.file.Files; +import java.nio.file.Paths; import java.io.*; import java.lang.ref.WeakReference; import java.nio.ByteBuffer; @@ -730,7 +732,7 @@ public abstract class SSTableReader extends SSTable implements SelfRefCounted<SS */ private void loadBloomFilter() throws IOException { - try (DataInputStream stream = new DataInputStream(new BufferedInputStream(new FileInputStream(descriptor.filenameFor(Component.FILTER))))) + try (DataInputStream stream = new DataInputStream(new BufferedInputStream(Files.newInputStream(Paths.get(descriptor.filenameFor(Component.FILTER)))))) { bf = FilterFactory.deserialize(stream, true); } @@ -871,7 +873,7 @@ public abstract class SSTableReader extends SSTable implements SelfRefCounted<SS try { TableMetadata metadata = metadata(); - iStream = new DataInputStream(new FileInputStream(summariesFile)); + iStream = new DataInputStream(Files.newInputStream(summariesFile.toPath())); indexSummary = IndexSummary.serializer.deserialize( iStream, getPartitioner(), metadata.params.minIndexInterval, metadata.params.maxIndexInterval); http://git-wip-us.apache.org/repos/asf/cassandra/blob/74bdf633/src/java/org/apache/cassandra/security/JKSKeyProvider.java ---------------------------------------------------------------------- diff --git a/src/java/org/apache/cassandra/security/JKSKeyProvider.java b/src/java/org/apache/cassandra/security/JKSKeyProvider.java index db7a2b9..cea7b23 100644 --- a/src/java/org/apache/cassandra/security/JKSKeyProvider.java +++ b/src/java/org/apache/cassandra/security/JKSKeyProvider.java @@ -17,7 +17,9 @@ */ package org.apache.cassandra.security; -import java.io.FileInputStream; +import java.nio.file.Files; +import java.nio.file.Paths; +import java.io.InputStream; import java.io.IOException; import java.security.Key; import java.security.KeyStore; @@ -46,7 +48,7 @@ public class JKSKeyProvider implements KeyProvider { this.options = options; logger.info("initializing keystore from file {}", options.get(PROP_KEYSTORE)); - try (FileInputStream inputStream = new FileInputStream(options.get(PROP_KEYSTORE))) + try (InputStream inputStream = Files.newInputStream(Paths.get(options.get(PROP_KEYSTORE)))) { store = KeyStore.getInstance(options.get(PROP_KEYSTORE_TYPE)); store.load(inputStream, options.get(PROP_KEYSTORE_PW).toCharArray()); http://git-wip-us.apache.org/repos/asf/cassandra/blob/74bdf633/src/java/org/apache/cassandra/security/SSLFactory.java ---------------------------------------------------------------------- diff --git a/src/java/org/apache/cassandra/security/SSLFactory.java b/src/java/org/apache/cassandra/security/SSLFactory.java index 7216e2c..33c1ad6 100644 --- a/src/java/org/apache/cassandra/security/SSLFactory.java +++ b/src/java/org/apache/cassandra/security/SSLFactory.java @@ -17,8 +17,9 @@ */ package org.apache.cassandra.security; - -import java.io.FileInputStream; +import java.nio.file.Files; +import java.nio.file.Paths; +import java.io.InputStream; import java.io.IOException; import java.net.InetAddress; import java.net.InetSocketAddress; @@ -155,8 +156,8 @@ public final class SSLFactory @SuppressWarnings("resource") public static SSLContext createSSLContext(EncryptionOptions options, boolean buildTruststore) throws IOException { - FileInputStream tsf = null; - FileInputStream ksf = null; + InputStream tsf = null; + InputStream ksf = null; SSLContext ctx; try { @@ -165,7 +166,7 @@ public final class SSLFactory if(buildTruststore) { - tsf = new FileInputStream(options.truststore); + tsf = Files.newInputStream(Paths.get(options.truststore)); TrustManagerFactory tmf = TrustManagerFactory.getInstance(options.algorithm); KeyStore ts = KeyStore.getInstance(options.store_type); ts.load(tsf, options.truststore_password.toCharArray()); @@ -173,7 +174,7 @@ public final class SSLFactory trustManagers = tmf.getTrustManagers(); } - ksf = new FileInputStream(options.keystore); + ksf = Files.newInputStream(Paths.get((options.keystore))); KeyManagerFactory kmf = KeyManagerFactory.getInstance(options.algorithm); KeyStore ks = KeyStore.getInstance(options.store_type); ks.load(ksf, options.keystore_password.toCharArray()); http://git-wip-us.apache.org/repos/asf/cassandra/blob/74bdf633/src/java/org/apache/cassandra/tools/SSTableMetadataViewer.java ---------------------------------------------------------------------- diff --git a/src/java/org/apache/cassandra/tools/SSTableMetadataViewer.java b/src/java/org/apache/cassandra/tools/SSTableMetadataViewer.java index 2f6aca0..65e6ece 100644 --- a/src/java/org/apache/cassandra/tools/SSTableMetadataViewer.java +++ b/src/java/org/apache/cassandra/tools/SSTableMetadataViewer.java @@ -19,6 +19,7 @@ package org.apache.cassandra.tools; import java.io.*; import java.nio.ByteBuffer; +import java.nio.file.Files; import java.util.Arrays; import java.util.EnumSet; import java.util.List; @@ -214,7 +215,7 @@ public class SSTableMetadataViewer if (!summariesFile.exists()) return; - try (DataInputStream iStream = new DataInputStream(new FileInputStream(summariesFile))) + try (DataInputStream iStream = new DataInputStream(Files.newInputStream(summariesFile.toPath()))) { Pair<DecoratedKey, DecoratedKey> firstLast = new IndexSummary.IndexSummarySerializer().deserializeFirstLastKey(iStream, partitioner); out.printf("First token: %s (key=%s)%n", firstLast.left.getToken(), keyType.getString(firstLast.left.getKey())); http://git-wip-us.apache.org/repos/asf/cassandra/blob/74bdf633/tools/stress/src/org/apache/cassandra/stress/StressGraph.java ---------------------------------------------------------------------- diff --git a/tools/stress/src/org/apache/cassandra/stress/StressGraph.java b/tools/stress/src/org/apache/cassandra/stress/StressGraph.java index 6729a28..7a865b4 100644 --- a/tools/stress/src/org/apache/cassandra/stress/StressGraph.java +++ b/tools/stress/src/org/apache/cassandra/stress/StressGraph.java @@ -148,7 +148,7 @@ public class StressGraph currentThreadCount = tc.group(2); } } - + // Detect mode changes if (line.equals(StressMetrics.HEAD)) { @@ -232,7 +232,7 @@ public class StressGraph private JSONObject createJSONStats(JSONObject json) { - try (InputStream logStream = new FileInputStream(stressSettings.graph.temporaryLogFile)) + try (InputStream logStream = Files.newInputStream(stressSettings.graph.temporaryLogFile.toPath())) { JSONArray stats; if (json == null) http://git-wip-us.apache.org/repos/asf/cassandra/blob/74bdf633/tools/stress/src/org/apache/cassandra/stress/settings/SettingsNode.java ---------------------------------------------------------------------- diff --git a/tools/stress/src/org/apache/cassandra/stress/settings/SettingsNode.java b/tools/stress/src/org/apache/cassandra/stress/settings/SettingsNode.java index ef20092..a029162 100644 --- a/tools/stress/src/org/apache/cassandra/stress/settings/SettingsNode.java +++ b/tools/stress/src/org/apache/cassandra/stress/settings/SettingsNode.java @@ -1,6 +1,6 @@ package org.apache.cassandra.stress.settings; /* - * + * * 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 @@ -8,19 +8,21 @@ package org.apache.cassandra.stress.settings; * 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. - * + * */ +import java.nio.file.Files; +import java.nio.file.Paths; import java.io.*; import java.net.InetAddress; import java.net.InetSocketAddress; @@ -44,7 +46,7 @@ public class SettingsNode implements Serializable { String node; List<String> tmpNodes = new ArrayList<>(); - try (BufferedReader in = new BufferedReader(new InputStreamReader(new FileInputStream(options.file.value())))) + try (BufferedReader in = new BufferedReader(new InputStreamReader(Files.newInputStream(Paths.get(options.file.value()))))) { while ((node = in.readLine()) != null) {