Skip to content

Commit

Permalink
[FLINK-6920] Remove minor guava usages
Browse files Browse the repository at this point in the history
This closes apache#4124.
  • Loading branch information
zentol committed Jun 23, 2017
1 parent 72b0ae0 commit 3424924
Show file tree
Hide file tree
Showing 9 changed files with 25 additions and 28 deletions.
Original file line number Diff line number Diff line change
Expand Up @@ -25,7 +25,6 @@
import org.apache.flink.streaming.connectors.elasticsearch.testutils.SourceSinkDataTestKit;
import org.apache.flink.streaming.connectors.elasticsearch.util.ElasticsearchUtils;

import com.google.common.collect.Lists;
import org.elasticsearch.action.index.IndexRequest;
import org.elasticsearch.client.Client;
import org.elasticsearch.client.Requests;
Expand All @@ -34,6 +33,7 @@
import org.junit.Test;

import java.net.InetSocketAddress;
import java.util.ArrayList;
import java.util.HashMap;
import java.util.List;
import java.util.Map;
Expand Down Expand Up @@ -113,7 +113,7 @@ public void testDeprecatedIndexRequestBuilderVariant() throws Exception {
userConfig.put("cluster.name", CLUSTER_NAME);
userConfig.put("node.local", "true");

List<TransportAddress> transports = Lists.newArrayList();
List<TransportAddress> transports = new ArrayList<>();
transports.add(new LocalTransportAddress("1"));

source.addSink(new ElasticsearchSink<>(
Expand Down Expand Up @@ -146,7 +146,7 @@ protected <T> ElasticsearchSinkBase<T> createElasticsearchSinkForEmbeddedNode(
// LocalTransportAddress to connect to a local embedded node
userConfig.put("node.local", "true");

List<TransportAddress> transports = Lists.newArrayList();
List<TransportAddress> transports = new ArrayList<>();
transports.add(new LocalTransportAddress("1"));

return new ElasticsearchSink<>(
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -26,7 +26,6 @@
import org.apache.flink.test.checkpointing.StreamFaultToleranceTestBase;
import org.apache.flink.util.NetUtils;

import com.google.common.collect.Sets;
import org.apache.hadoop.conf.Configuration;
import org.apache.hadoop.fs.FSDataInputStream;
import org.apache.hadoop.fs.LocatedFileStatus;
Expand Down Expand Up @@ -139,7 +138,7 @@ public void postSubmit() throws Exception {
// Keep a set of the message IDs that we read. The size must equal the read count and
// the NUM_STRINGS. If numRead is bigger than the size of the set we have seen some
// elements twice.
Set<Integer> readNumbers = Sets.newHashSet();
Set<Integer> readNumbers = new HashSet<>();

HashSet<String> uniqMessagesRead = new HashSet<>();
HashSet<String> messagesInCommittedFiles = new HashSet<>();
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -26,7 +26,6 @@
import org.apache.flink.test.checkpointing.StreamFaultToleranceTestBase;
import org.apache.flink.util.NetUtils;

import com.google.common.collect.Sets;
import org.apache.hadoop.conf.Configuration;
import org.apache.hadoop.fs.FSDataInputStream;
import org.apache.hadoop.fs.LocatedFileStatus;
Expand Down Expand Up @@ -136,7 +135,7 @@ public void postSubmit() throws Exception {
// Keep a set of the message IDs that we read. The size must equal the read count and
// the NUM_STRINGS. If numRead is bigger than the size of the set we have seen some
// elements twice.
Set<Integer> readNumbers = Sets.newHashSet();
Set<Integer> readNumbers = new HashSet<>();

HashSet<String> uniqMessagesRead = new HashSet<>();
HashSet<String> messagesInCommittedFiles = new HashSet<>();
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -22,9 +22,8 @@
import org.apache.flink.graph.examples.data.EuclideanGraphData;
import org.apache.flink.test.util.MultipleProgramsTestBase;
import org.apache.flink.test.util.TestBaseUtils;
import org.apache.flink.util.FileUtils;

import com.google.common.base.Charsets;
import com.google.common.io.Files;
import org.junit.After;
import org.junit.Before;
import org.junit.Rule;
Expand Down Expand Up @@ -60,10 +59,10 @@ public EuclideanGraphWeighingITCase(TestExecutionMode mode) {
public void before() throws Exception {
resultPath = tempFolder.newFile().toURI().toString();
File verticesFile = tempFolder.newFile();
Files.write(EuclideanGraphData.VERTICES, verticesFile, Charsets.UTF_8);
FileUtils.writeFileUtf8(verticesFile, EuclideanGraphData.VERTICES);

File edgesFile = tempFolder.newFile();
Files.write(EuclideanGraphData.EDGES, edgesFile, Charsets.UTF_8);
FileUtils.writeFileUtf8(edgesFile, EuclideanGraphData.EDGES);

verticesPath = verticesFile.toURI().toString();
edgesPath = edgesFile.toURI().toString();
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -29,9 +29,8 @@
import org.apache.flink.graph.spargel.ScatterGatherConfiguration;
import org.apache.flink.test.util.MultipleProgramsTestBase;
import org.apache.flink.test.util.TestBaseUtils;
import org.apache.flink.util.FileUtils;

import com.google.common.base.Charsets;
import com.google.common.io.Files;
import org.junit.After;
import org.junit.Before;
import org.junit.Rule;
Expand Down Expand Up @@ -69,13 +68,13 @@ public IncrementalSSSPITCase(TestExecutionMode mode) {
public void before() throws Exception {
resultPath = tempFolder.newFile().toURI().toString();
File verticesFile = tempFolder.newFile();
Files.write(IncrementalSSSPData.VERTICES, verticesFile, Charsets.UTF_8);
FileUtils.writeFileUtf8(verticesFile, IncrementalSSSPData.VERTICES);

File edgesFile = tempFolder.newFile();
Files.write(IncrementalSSSPData.EDGES, edgesFile, Charsets.UTF_8);
FileUtils.writeFileUtf8(edgesFile, IncrementalSSSPData.EDGES);

File edgesInSSSPFile = tempFolder.newFile();
Files.write(IncrementalSSSPData.EDGES_IN_SSSP, edgesInSSSPFile, Charsets.UTF_8);
FileUtils.writeFileUtf8(edgesInSSSPFile, IncrementalSSSPData.EDGES_IN_SSSP);

verticesPath = verticesFile.toURI().toString();
edgesPath = edgesFile.toURI().toString();
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -22,9 +22,8 @@
import org.apache.flink.graph.examples.data.MusicProfilesData;
import org.apache.flink.test.util.MultipleProgramsTestBase;
import org.apache.flink.test.util.TestBaseUtils;
import org.apache.flink.util.FileUtils;

import com.google.common.base.Charsets;
import com.google.common.io.Files;
import org.junit.After;
import org.junit.Assert;
import org.junit.Before;
Expand Down Expand Up @@ -67,11 +66,11 @@ public void before() throws Exception {
communitiesResultPath = tempFolder.newFile().toURI().toString();

File tripletsFile = tempFolder.newFile();
Files.write(MusicProfilesData.USER_SONG_TRIPLETS, tripletsFile, Charsets.UTF_8);
FileUtils.writeFileUtf8(tripletsFile, MusicProfilesData.USER_SONG_TRIPLETS);
tripletsPath = tripletsFile.toURI().toString();

File mismatchesFile = tempFolder.newFile();
Files.write(MusicProfilesData.MISMATCHES, mismatchesFile, Charsets.UTF_8);
FileUtils.writeFileUtf8(mismatchesFile, MusicProfilesData.MISMATCHES);
mismatchesPath = mismatchesFile.toURI().toString();
}

Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -24,9 +24,8 @@
import org.apache.flink.graph.examples.data.SingleSourceShortestPathsData;
import org.apache.flink.test.util.MultipleProgramsTestBase;
import org.apache.flink.test.util.TestBaseUtils;
import org.apache.flink.util.FileUtils;

import com.google.common.base.Charsets;
import com.google.common.io.Files;
import org.junit.After;
import org.junit.Before;
import org.junit.Rule;
Expand Down Expand Up @@ -61,7 +60,7 @@ public void before() throws Exception {
resultPath = tempFolder.newFile().toURI().toString();

File edgesFile = tempFolder.newFile();
Files.write(SingleSourceShortestPathsData.EDGES, edgesFile, Charsets.UTF_8);
FileUtils.writeFileUtf8(edgesFile, SingleSourceShortestPathsData.EDGES);
edgesPath = edgesFile.toURI().toString();
}

Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -19,8 +19,8 @@
package org.apache.flink.graph.scala.test.operations

import java.io.{File, FileOutputStream, IOException, OutputStreamWriter}
import java.nio.charset.StandardCharsets

import com.google.common.base.Charsets
import org.apache.flink.api.common.functions.MapFunction
import org.apache.flink.api.scala._
import org.apache.flink.core.fs.{FileInputSplit, Path}
Expand Down Expand Up @@ -195,7 +195,7 @@ MultipleProgramsTestBase(mode) {
val tempFile = File.createTempFile("test_contents", "tmp")
tempFile.deleteOnExit()

val wrt = new OutputStreamWriter(new FileOutputStream(tempFile), Charsets.UTF_8)
val wrt = new OutputStreamWriter(new FileOutputStream(tempFile), StandardCharsets.UTF_8)
wrt.write(content)
wrt.close()

Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -21,9 +21,8 @@
import org.apache.flink.configuration.Configuration;
import org.apache.flink.runtime.akka.AkkaUtils;
import org.apache.flink.runtime.minicluster.LocalFlinkMiniCluster;
import org.apache.flink.util.FileUtils;

import com.google.common.base.Charsets;
import com.google.common.io.Files;
import org.junit.ClassRule;
import org.junit.rules.TemporaryFolder;

Expand Down Expand Up @@ -112,7 +111,11 @@ public String getTempFilePath(String fileName) throws IOException {

public String createTempFile(String fileName, String contents) throws IOException {
File f = createAndRegisterTempFile(fileName);
Files.write(contents, f, Charsets.UTF_8);
if (!f.getParentFile().exists()) {
f.getParentFile().mkdirs();
}
f.createNewFile();
FileUtils.writeFileUtf8(f, contents);
return f.toURI().toString();
}

Expand Down

0 comments on commit 3424924

Please sign in to comment.