Repository: flink Updated Branches: refs/heads/master 05c765e57 -> 43bd6f6e4
[FLINK-3821] [dataSet] Remove Guava usage in flink-java non-test files. - replaced CharSets with StandardCharsets - added checkElementIndex to Flink Preconditions - replaced Guava Preconditions with Flink Preconditions - removed single usages Ints.max() and Joiner() This closes #1938 Project: http://git-wip-us.apache.org/repos/asf/flink/repo Commit: http://git-wip-us.apache.org/repos/asf/flink/commit/15f52112 Tree: http://git-wip-us.apache.org/repos/asf/flink/tree/15f52112 Diff: http://git-wip-us.apache.org/repos/asf/flink/diff/15f52112 Branch: refs/heads/master Commit: 15f52112af1e3cca86094b5c6c1ef31157e073df Parents: 05c765e Author: zentol <ches...@apache.org> Authored: Wed Apr 27 14:43:17 2016 +0200 Committer: Fabian Hueske <fhue...@apache.org> Committed: Tue May 10 18:58:11 2016 +0200 ---------------------------------------------------------------------- .../java/org/apache/flink/util/Preconditions.java | 18 +++++++++++++++++- flink-java/pom.xml | 13 +++++++------ .../java/org/apache/flink/api/java/DataSet.java | 2 +- .../flink/api/java/ExecutionEnvironment.java | 16 +++++++--------- .../hadoop/mapreduce/HadoopInputFormatBase.java | 11 +++++------ .../apache/flink/api/java/io/CsvInputFormat.java | 7 ++++--- .../org/apache/flink/api/java/io/CsvReader.java | 3 +-- .../flink/api/java/io/PojoCsvInputFormat.java | 2 +- .../flink/api/java/io/TextValueInputFormat.java | 4 ++-- .../api/java/operators/AggregateOperator.java | 3 +-- .../flink/api/java/operators/CrossOperator.java | 3 +-- .../flink/api/java/operators/DeltaIteration.java | 3 +-- .../flink/api/java/operators/JoinOperator.java | 3 +-- .../flink/api/java/operators/ProjectOperator.java | 3 +-- .../flink/api/java/operators/SortedGrouping.java | 3 +-- .../api/java/operators/UnsortedGrouping.java | 3 +-- .../translation/CombineToGroupCombineWrapper.java | 2 +- .../RichCombineToGroupCombineWrapper.java | 2 +- .../flink/api/java/sampling/BernoulliSampler.java | 2 +- .../flink/api/java/sampling/PoissonSampler.java | 2 +- .../sampling/ReservoirSamplerWithReplacement.java | 2 +- .../ReservoirSamplerWithoutReplacement.java | 2 +- .../apache/flink/api/java/utils/DataSetUtils.java | 7 +++++-- .../flink/api/java/utils/ParameterTool.java | 2 +- .../flink/api/java/io/CsvInputFormatTest.java | 5 ++--- .../api/java/sampling/RandomSamplerTest.java | 3 +-- .../flink/api/java/tuple/TupleGenerator.java | 8 ++++---- 27 files changed, 71 insertions(+), 63 deletions(-) ---------------------------------------------------------------------- http://git-wip-us.apache.org/repos/asf/flink/blob/15f52112/flink-core/src/main/java/org/apache/flink/util/Preconditions.java ---------------------------------------------------------------------- diff --git a/flink-core/src/main/java/org/apache/flink/util/Preconditions.java b/flink-core/src/main/java/org/apache/flink/util/Preconditions.java index a9bd166..25fefe5 100644 --- a/flink-core/src/main/java/org/apache/flink/util/Preconditions.java +++ b/flink-core/src/main/java/org/apache/flink/util/Preconditions.java @@ -218,6 +218,22 @@ public final class Preconditions { } } + /** + * Ensures that the given index is valid for an array, list or string of the given size. + * + * @param index index to check + * @param size size of the array, list or string + * + * @throws IllegalArgumentException Thrown, if size is negative. + * @throws IndexOutOfBoundsException Thrown, if the index negative or greater than or equal to size + */ + public static void checkElementIndex(int index, int size) { + checkArgument(size >= 0, "Size was negative."); + if (index < 0 || index >= size) { + throw new IndexOutOfBoundsException("Index: " + index + ", Size: " + size); + } + } + // ------------------------------------------------------------------------ // Utilities // ------------------------------------------------------------------------ @@ -265,4 +281,4 @@ public final class Preconditions { /** Private constructor to prevent instantiation */ private Preconditions() {} -} \ No newline at end of file +} http://git-wip-us.apache.org/repos/asf/flink/blob/15f52112/flink-java/pom.xml ---------------------------------------------------------------------- diff --git a/flink-java/pom.xml b/flink-java/pom.xml index 5150c5e..a3689e7 100644 --- a/flink-java/pom.xml +++ b/flink-java/pom.xml @@ -54,16 +54,17 @@ under the License. </dependency> <dependency> - <groupId>com.google.guava</groupId> - <artifactId>guava</artifactId> - <version>${guava.version}</version> - </dependency> - - <dependency> <groupId>org.apache.commons</groupId> <artifactId>commons-math3</artifactId> <!-- managed version --> </dependency> + + <dependency> + <groupId>com.google.guava</groupId> + <artifactId>guava</artifactId> + <version>${guava.version}</version> + <scope>test</scope> + </dependency> <dependency> <groupId>org.apache.flink</groupId> http://git-wip-us.apache.org/repos/asf/flink/blob/15f52112/flink-java/src/main/java/org/apache/flink/api/java/DataSet.java ---------------------------------------------------------------------- diff --git a/flink-java/src/main/java/org/apache/flink/api/java/DataSet.java b/flink-java/src/main/java/org/apache/flink/api/java/DataSet.java index b186c3c..530de4b 100644 --- a/flink-java/src/main/java/org/apache/flink/api/java/DataSet.java +++ b/flink-java/src/main/java/org/apache/flink/api/java/DataSet.java @@ -18,7 +18,6 @@ package org.apache.flink.api.java; -import com.google.common.base.Preconditions; import org.apache.flink.annotation.PublicEvolving; import org.apache.flink.annotation.Public; import org.apache.flink.api.common.InvalidProgramException; @@ -87,6 +86,7 @@ import org.apache.flink.api.java.typeutils.TypeExtractor; import org.apache.flink.core.fs.FileSystem.WriteMode; import org.apache.flink.core.fs.Path; import org.apache.flink.util.AbstractID; +import org.apache.flink.util.Preconditions; import java.io.IOException; import java.util.ArrayList; http://git-wip-us.apache.org/repos/asf/flink/blob/15f52112/flink-java/src/main/java/org/apache/flink/api/java/ExecutionEnvironment.java ---------------------------------------------------------------------- diff --git a/flink-java/src/main/java/org/apache/flink/api/java/ExecutionEnvironment.java b/flink-java/src/main/java/org/apache/flink/api/java/ExecutionEnvironment.java index d108d3e..be841ac 100644 --- a/flink-java/src/main/java/org/apache/flink/api/java/ExecutionEnvironment.java +++ b/flink-java/src/main/java/org/apache/flink/api/java/ExecutionEnvironment.java @@ -20,9 +20,6 @@ package org.apache.flink.api.java; import com.esotericsoftware.kryo.Serializer; -import com.google.common.base.Joiner; -import com.google.common.base.Preconditions; - import org.apache.flink.annotation.PublicEvolving; import org.apache.flink.annotation.Internal; import org.apache.flink.annotation.Public; @@ -60,6 +57,7 @@ import org.apache.flink.configuration.Configuration; import org.apache.flink.core.fs.Path; import org.apache.flink.types.StringValue; import org.apache.flink.util.NumberSequenceIterator; +import org.apache.flink.util.Preconditions; import org.apache.flink.util.SplittableIterator; import org.apache.flink.util.Visitor; @@ -1090,12 +1088,12 @@ public abstract class ExecutionEnvironment { } if(LOG.isDebugEnabled()) { - LOG.debug("Registered Kryo types: {}", Joiner.on(',').join(config.getRegisteredKryoTypes())); - LOG.debug("Registered Kryo with Serializers types: {}", Joiner.on(',').join(config.getRegisteredTypesWithKryoSerializers().entrySet())); - LOG.debug("Registered Kryo with Serializer Classes types: {}", Joiner.on(',').join(config.getRegisteredTypesWithKryoSerializerClasses().entrySet())); - LOG.debug("Registered Kryo default Serializers: {}", Joiner.on(',').join(config.getDefaultKryoSerializers().entrySet())); - LOG.debug("Registered Kryo default Serializers Classes {}", Joiner.on(',').join(config.getDefaultKryoSerializerClasses().entrySet())); - LOG.debug("Registered POJO types: {}", Joiner.on(',').join(config.getRegisteredPojoTypes())); + LOG.debug("Registered Kryo types: {}", config.getRegisteredKryoTypes().toString()); + LOG.debug("Registered Kryo with Serializers types: {}", config.getRegisteredTypesWithKryoSerializers().entrySet().toString()); + LOG.debug("Registered Kryo with Serializer Classes types: {}", config.getRegisteredTypesWithKryoSerializerClasses().entrySet().toString()); + LOG.debug("Registered Kryo default Serializers: {}", config.getDefaultKryoSerializers().entrySet().toString()); + LOG.debug("Registered Kryo default Serializers Classes {}", config.getDefaultKryoSerializerClasses().entrySet().toString()); + LOG.debug("Registered POJO types: {}", config.getRegisteredPojoTypes().toString()); // print information about static code analysis LOG.debug("Static code analysis mode: {}", config.getCodeAnalysisMode()); http://git-wip-us.apache.org/repos/asf/flink/blob/15f52112/flink-java/src/main/java/org/apache/flink/api/java/hadoop/mapreduce/HadoopInputFormatBase.java ---------------------------------------------------------------------- diff --git a/flink-java/src/main/java/org/apache/flink/api/java/hadoop/mapreduce/HadoopInputFormatBase.java b/flink-java/src/main/java/org/apache/flink/api/java/hadoop/mapreduce/HadoopInputFormatBase.java index 73b11eb..0335c23 100644 --- a/flink-java/src/main/java/org/apache/flink/api/java/hadoop/mapreduce/HadoopInputFormatBase.java +++ b/flink-java/src/main/java/org/apache/flink/api/java/hadoop/mapreduce/HadoopInputFormatBase.java @@ -30,6 +30,7 @@ import org.apache.flink.core.fs.FileStatus; import org.apache.flink.core.fs.FileSystem; import org.apache.flink.core.fs.Path; import org.apache.flink.core.io.InputSplitAssigner; +import org.apache.flink.util.Preconditions; import org.apache.hadoop.conf.Configurable; import org.apache.hadoop.mapreduce.Job; import org.apache.hadoop.mapreduce.JobContext; @@ -49,8 +50,6 @@ import java.io.ObjectOutputStream; import java.util.ArrayList; import java.util.List; -import static com.google.common.base.Preconditions.checkNotNull; - /** * Base class shared between the Java and Scala API of Flink */ @@ -81,10 +80,10 @@ public abstract class HadoopInputFormatBase<K, V, T> extends HadoopInputFormatCo protected boolean hasNext; public HadoopInputFormatBase(org.apache.hadoop.mapreduce.InputFormat<K, V> mapreduceInputFormat, Class<K> key, Class<V> value, Job job) { - super(checkNotNull(job, "Job can not be null").getCredentials()); - this.mapreduceInputFormat = checkNotNull(mapreduceInputFormat); - this.keyClass = checkNotNull(key); - this.valueClass = checkNotNull(value); + super(Preconditions.checkNotNull(job, "Job can not be null").getCredentials()); + this.mapreduceInputFormat = Preconditions.checkNotNull(mapreduceInputFormat); + this.keyClass = Preconditions.checkNotNull(key); + this.valueClass = Preconditions.checkNotNull(value); this.configuration = job.getConfiguration(); HadoopUtils.mergeHadoopConf(configuration); } http://git-wip-us.apache.org/repos/asf/flink/blob/15f52112/flink-java/src/main/java/org/apache/flink/api/java/io/CsvInputFormat.java ---------------------------------------------------------------------- diff --git a/flink-java/src/main/java/org/apache/flink/api/java/io/CsvInputFormat.java b/flink-java/src/main/java/org/apache/flink/api/java/io/CsvInputFormat.java index 4105305..2213d9f 100644 --- a/flink-java/src/main/java/org/apache/flink/api/java/io/CsvInputFormat.java +++ b/flink-java/src/main/java/org/apache/flink/api/java/io/CsvInputFormat.java @@ -18,12 +18,11 @@ package org.apache.flink.api.java.io; -import com.google.common.base.Preconditions; -import com.google.common.primitives.Ints; import org.apache.flink.annotation.Internal; import org.apache.flink.api.common.io.GenericCsvInputFormat; import org.apache.flink.core.fs.FileInputSplit; import org.apache.flink.types.parser.FieldParser; +import org.apache.flink.util.Preconditions; import java.io.IOException; import org.apache.flink.core.fs.Path; @@ -133,13 +132,15 @@ public abstract class CsvInputFormat<OUT> extends GenericCsvInputFormat<OUT> { protected static boolean[] toBooleanMask(int[] sourceFieldIndices) { Preconditions.checkNotNull(sourceFieldIndices); + int max = 0; for (int i : sourceFieldIndices) { if (i < 0) { throw new IllegalArgumentException("Field indices must not be smaller than zero."); } + max = Math.max(i, max); } - boolean[] includedMask = new boolean[Ints.max(sourceFieldIndices) + 1]; + boolean[] includedMask = new boolean[max + 1]; // check if we support parsers for these types for (int i = 0; i < sourceFieldIndices.length; i++) { http://git-wip-us.apache.org/repos/asf/flink/blob/15f52112/flink-java/src/main/java/org/apache/flink/api/java/io/CsvReader.java ---------------------------------------------------------------------- diff --git a/flink-java/src/main/java/org/apache/flink/api/java/io/CsvReader.java b/flink-java/src/main/java/org/apache/flink/api/java/io/CsvReader.java index 3d656a4..8be5dc2 100644 --- a/flink-java/src/main/java/org/apache/flink/api/java/io/CsvReader.java +++ b/flink-java/src/main/java/org/apache/flink/api/java/io/CsvReader.java @@ -30,13 +30,12 @@ import org.apache.flink.api.java.typeutils.PojoTypeInfo; import org.apache.flink.api.java.typeutils.TupleTypeInfo; import org.apache.flink.api.java.typeutils.TypeExtractor; import org.apache.flink.core.fs.Path; +import org.apache.flink.util.Preconditions; //CHECKSTYLE.OFF: AvoidStarImport - Needed for TupleGenerator import org.apache.flink.api.java.tuple.*; //CHECKSTYLE.ON: AvoidStarImport -import com.google.common.base.Preconditions; - /** * A builder class to instantiate a CSV parsing data source. The CSV reader configures the field types, * the delimiters (row and field), the fields that should be included or skipped, and other flags http://git-wip-us.apache.org/repos/asf/flink/blob/15f52112/flink-java/src/main/java/org/apache/flink/api/java/io/PojoCsvInputFormat.java ---------------------------------------------------------------------- diff --git a/flink-java/src/main/java/org/apache/flink/api/java/io/PojoCsvInputFormat.java b/flink-java/src/main/java/org/apache/flink/api/java/io/PojoCsvInputFormat.java index 26a771f..990e9e6 100644 --- a/flink-java/src/main/java/org/apache/flink/api/java/io/PojoCsvInputFormat.java +++ b/flink-java/src/main/java/org/apache/flink/api/java/io/PojoCsvInputFormat.java @@ -17,11 +17,11 @@ */ package org.apache.flink.api.java.io; -import com.google.common.base.Preconditions; import org.apache.flink.annotation.Internal; import org.apache.flink.api.java.typeutils.PojoTypeInfo; import org.apache.flink.core.fs.FileInputSplit; import org.apache.flink.core.fs.Path; +import org.apache.flink.util.Preconditions; import java.io.IOException; import java.lang.reflect.Field; http://git-wip-us.apache.org/repos/asf/flink/blob/15f52112/flink-java/src/main/java/org/apache/flink/api/java/io/TextValueInputFormat.java ---------------------------------------------------------------------- diff --git a/flink-java/src/main/java/org/apache/flink/api/java/io/TextValueInputFormat.java b/flink-java/src/main/java/org/apache/flink/api/java/io/TextValueInputFormat.java index fbfcd8d..a0d20d6 100644 --- a/flink-java/src/main/java/org/apache/flink/api/java/io/TextValueInputFormat.java +++ b/flink-java/src/main/java/org/apache/flink/api/java/io/TextValueInputFormat.java @@ -23,9 +23,9 @@ import java.nio.CharBuffer; import java.nio.charset.CharacterCodingException; import java.nio.charset.Charset; import java.nio.charset.CharsetDecoder; +import java.nio.charset.StandardCharsets; import java.util.Arrays; -import com.google.common.base.Charsets; import org.apache.flink.annotation.PublicEvolving; import org.apache.flink.api.common.io.DelimitedInputFormat; import org.apache.flink.configuration.Configuration; @@ -85,7 +85,7 @@ public class TextValueInputFormat extends DelimitedInputFormat<StringValue> { throw new RuntimeException("Unsupported charset: " + charsetName); } - if (charsetName.equalsIgnoreCase(Charsets.US_ASCII.name())) { + if (charsetName.equalsIgnoreCase(StandardCharsets.US_ASCII.name())) { ascii = true; } http://git-wip-us.apache.org/repos/asf/flink/blob/15f52112/flink-java/src/main/java/org/apache/flink/api/java/operators/AggregateOperator.java ---------------------------------------------------------------------- diff --git a/flink-java/src/main/java/org/apache/flink/api/java/operators/AggregateOperator.java b/flink-java/src/main/java/org/apache/flink/api/java/operators/AggregateOperator.java index 20aabd9..26cf054 100644 --- a/flink-java/src/main/java/org/apache/flink/api/java/operators/AggregateOperator.java +++ b/flink-java/src/main/java/org/apache/flink/api/java/operators/AggregateOperator.java @@ -40,8 +40,7 @@ import org.apache.flink.api.java.tuple.Tuple; import org.apache.flink.api.java.typeutils.TupleTypeInfoBase; import org.apache.flink.configuration.Configuration; import org.apache.flink.util.Collector; - -import com.google.common.base.Preconditions; +import org.apache.flink.util.Preconditions; /** * This operator represents the application of a "aggregate" operation on a data set, and the http://git-wip-us.apache.org/repos/asf/flink/blob/15f52112/flink-java/src/main/java/org/apache/flink/api/java/operators/CrossOperator.java ---------------------------------------------------------------------- diff --git a/flink-java/src/main/java/org/apache/flink/api/java/operators/CrossOperator.java b/flink-java/src/main/java/org/apache/flink/api/java/operators/CrossOperator.java index e69e16e..36e6c1c 100644 --- a/flink-java/src/main/java/org/apache/flink/api/java/operators/CrossOperator.java +++ b/flink-java/src/main/java/org/apache/flink/api/java/operators/CrossOperator.java @@ -36,12 +36,11 @@ import org.apache.flink.api.java.Utils; import org.apache.flink.api.java.functions.SemanticPropUtil; import org.apache.flink.api.java.typeutils.TupleTypeInfo; import org.apache.flink.api.java.typeutils.TypeExtractor; +import org.apache.flink.util.Preconditions; //CHECKSTYLE.OFF: AvoidStarImport - Needed for TupleGenerator import org.apache.flink.api.java.tuple.*; -import com.google.common.base.Preconditions; - /** * A {@link DataSet} that is the result of a Cross transformation. * http://git-wip-us.apache.org/repos/asf/flink/blob/15f52112/flink-java/src/main/java/org/apache/flink/api/java/operators/DeltaIteration.java ---------------------------------------------------------------------- diff --git a/flink-java/src/main/java/org/apache/flink/api/java/operators/DeltaIteration.java b/flink-java/src/main/java/org/apache/flink/api/java/operators/DeltaIteration.java index ccf88a6..d53b499 100644 --- a/flink-java/src/main/java/org/apache/flink/api/java/operators/DeltaIteration.java +++ b/flink-java/src/main/java/org/apache/flink/api/java/operators/DeltaIteration.java @@ -30,8 +30,7 @@ import org.apache.flink.api.common.operators.Keys; import org.apache.flink.api.common.typeinfo.TypeInformation; import org.apache.flink.api.java.DataSet; import org.apache.flink.api.java.ExecutionEnvironment; - -import com.google.common.base.Preconditions; +import org.apache.flink.util.Preconditions; /** * The DeltaIteration represents the start of a delta iteration. It is created from the DataSet that http://git-wip-us.apache.org/repos/asf/flink/blob/15f52112/flink-java/src/main/java/org/apache/flink/api/java/operators/JoinOperator.java ---------------------------------------------------------------------- diff --git a/flink-java/src/main/java/org/apache/flink/api/java/operators/JoinOperator.java b/flink-java/src/main/java/org/apache/flink/api/java/operators/JoinOperator.java index 1d1ec27..88c479b 100644 --- a/flink-java/src/main/java/org/apache/flink/api/java/operators/JoinOperator.java +++ b/flink-java/src/main/java/org/apache/flink/api/java/operators/JoinOperator.java @@ -20,8 +20,6 @@ package org.apache.flink.api.java.operators; import java.util.Arrays; -import com.google.common.base.Preconditions; - import org.apache.flink.annotation.Internal; import org.apache.flink.annotation.Public; import org.apache.flink.annotation.PublicEvolving; @@ -59,6 +57,7 @@ import org.apache.flink.api.java.operators.translation.WrappingFunction; import org.apache.flink.api.java.typeutils.TupleTypeInfo; import org.apache.flink.api.java.typeutils.TypeExtractor; import org.apache.flink.util.Collector; +import org.apache.flink.util.Preconditions; //CHECKSTYLE.OFF: AvoidStarImport - Needed for TupleGenerator import org.apache.flink.api.java.tuple.*; http://git-wip-us.apache.org/repos/asf/flink/blob/15f52112/flink-java/src/main/java/org/apache/flink/api/java/operators/ProjectOperator.java ---------------------------------------------------------------------- diff --git a/flink-java/src/main/java/org/apache/flink/api/java/operators/ProjectOperator.java b/flink-java/src/main/java/org/apache/flink/api/java/operators/ProjectOperator.java index d8a5835..6c27167 100644 --- a/flink-java/src/main/java/org/apache/flink/api/java/operators/ProjectOperator.java +++ b/flink-java/src/main/java/org/apache/flink/api/java/operators/ProjectOperator.java @@ -32,13 +32,12 @@ import org.apache.flink.api.java.DataSet; import org.apache.flink.api.java.functions.SemanticPropUtil; import org.apache.flink.api.java.operators.translation.PlanProjectOperator; import org.apache.flink.api.java.typeutils.TupleTypeInfo; +import org.apache.flink.util.Preconditions; //CHECKSTYLE.OFF: AvoidStarImport - Needed for TupleGenerator import org.apache.flink.api.java.tuple.*; //CHECKSTYLE.ON: AvoidStarImport -import com.google.common.base.Preconditions; - /** * This operator represents the application of a projection operation on a data set, and the * result data set produced by the function. http://git-wip-us.apache.org/repos/asf/flink/blob/15f52112/flink-java/src/main/java/org/apache/flink/api/java/operators/SortedGrouping.java ---------------------------------------------------------------------- diff --git a/flink-java/src/main/java/org/apache/flink/api/java/operators/SortedGrouping.java b/flink-java/src/main/java/org/apache/flink/api/java/operators/SortedGrouping.java index c4ae24b..11645df 100644 --- a/flink-java/src/main/java/org/apache/flink/api/java/operators/SortedGrouping.java +++ b/flink-java/src/main/java/org/apache/flink/api/java/operators/SortedGrouping.java @@ -35,8 +35,7 @@ import org.apache.flink.api.common.operators.Order; import org.apache.flink.api.java.DataSet; import org.apache.flink.api.common.operators.Keys.ExpressionKeys; import org.apache.flink.api.java.typeutils.TypeExtractor; - -import com.google.common.base.Preconditions; +import org.apache.flink.util.Preconditions; /** * SortedGrouping is an intermediate step for a transformation on a grouped and sorted DataSet.<br> http://git-wip-us.apache.org/repos/asf/flink/blob/15f52112/flink-java/src/main/java/org/apache/flink/api/java/operators/UnsortedGrouping.java ---------------------------------------------------------------------- diff --git a/flink-java/src/main/java/org/apache/flink/api/java/operators/UnsortedGrouping.java b/flink-java/src/main/java/org/apache/flink/api/java/operators/UnsortedGrouping.java index ed5754c..2ad133c 100644 --- a/flink-java/src/main/java/org/apache/flink/api/java/operators/UnsortedGrouping.java +++ b/flink-java/src/main/java/org/apache/flink/api/java/operators/UnsortedGrouping.java @@ -36,8 +36,7 @@ import org.apache.flink.api.java.functions.SelectByMaxFunction; import org.apache.flink.api.java.functions.SelectByMinFunction; import org.apache.flink.api.java.typeutils.TupleTypeInfo; import org.apache.flink.api.java.typeutils.TypeExtractor; - -import com.google.common.base.Preconditions; +import org.apache.flink.util.Preconditions; @Public public class UnsortedGrouping<T> extends Grouping<T> { http://git-wip-us.apache.org/repos/asf/flink/blob/15f52112/flink-java/src/main/java/org/apache/flink/api/java/operators/translation/CombineToGroupCombineWrapper.java ---------------------------------------------------------------------- diff --git a/flink-java/src/main/java/org/apache/flink/api/java/operators/translation/CombineToGroupCombineWrapper.java b/flink-java/src/main/java/org/apache/flink/api/java/operators/translation/CombineToGroupCombineWrapper.java index 3488403..408d4b3 100644 --- a/flink-java/src/main/java/org/apache/flink/api/java/operators/translation/CombineToGroupCombineWrapper.java +++ b/flink-java/src/main/java/org/apache/flink/api/java/operators/translation/CombineToGroupCombineWrapper.java @@ -17,12 +17,12 @@ */ package org.apache.flink.api.java.operators.translation; -import com.google.common.base.Preconditions; import org.apache.flink.annotation.Internal; import org.apache.flink.api.common.functions.CombineFunction; import org.apache.flink.api.common.functions.GroupCombineFunction; import org.apache.flink.api.common.functions.GroupReduceFunction; import org.apache.flink.util.Collector; +import org.apache.flink.util.Preconditions; /** * A wrapper the wraps a function that implements both {@link CombineFunction} and {@link GroupReduceFunction} interfaces http://git-wip-us.apache.org/repos/asf/flink/blob/15f52112/flink-java/src/main/java/org/apache/flink/api/java/operators/translation/RichCombineToGroupCombineWrapper.java ---------------------------------------------------------------------- diff --git a/flink-java/src/main/java/org/apache/flink/api/java/operators/translation/RichCombineToGroupCombineWrapper.java b/flink-java/src/main/java/org/apache/flink/api/java/operators/translation/RichCombineToGroupCombineWrapper.java index 680f36d..d8c54d6 100644 --- a/flink-java/src/main/java/org/apache/flink/api/java/operators/translation/RichCombineToGroupCombineWrapper.java +++ b/flink-java/src/main/java/org/apache/flink/api/java/operators/translation/RichCombineToGroupCombineWrapper.java @@ -17,7 +17,6 @@ */ package org.apache.flink.api.java.operators.translation; -import com.google.common.base.Preconditions; import org.apache.flink.api.common.functions.CombineFunction; import org.apache.flink.api.common.functions.GroupCombineFunction; import org.apache.flink.api.common.functions.GroupReduceFunction; @@ -25,6 +24,7 @@ import org.apache.flink.api.common.functions.RichGroupCombineFunction; import org.apache.flink.api.common.functions.RichGroupReduceFunction; import org.apache.flink.configuration.Configuration; import org.apache.flink.util.Collector; +import org.apache.flink.util.Preconditions; /** * A wrapper the wraps a function that implements both {@link CombineFunction} and {@link GroupReduceFunction} interfaces http://git-wip-us.apache.org/repos/asf/flink/blob/15f52112/flink-java/src/main/java/org/apache/flink/api/java/sampling/BernoulliSampler.java ---------------------------------------------------------------------- diff --git a/flink-java/src/main/java/org/apache/flink/api/java/sampling/BernoulliSampler.java b/flink-java/src/main/java/org/apache/flink/api/java/sampling/BernoulliSampler.java index cff5f1c..c370f9d 100644 --- a/flink-java/src/main/java/org/apache/flink/api/java/sampling/BernoulliSampler.java +++ b/flink-java/src/main/java/org/apache/flink/api/java/sampling/BernoulliSampler.java @@ -17,8 +17,8 @@ */ package org.apache.flink.api.java.sampling; -import com.google.common.base.Preconditions; import org.apache.flink.annotation.Internal; +import org.apache.flink.util.Preconditions; import org.apache.flink.util.XORShiftRandom; import java.util.Iterator; http://git-wip-us.apache.org/repos/asf/flink/blob/15f52112/flink-java/src/main/java/org/apache/flink/api/java/sampling/PoissonSampler.java ---------------------------------------------------------------------- diff --git a/flink-java/src/main/java/org/apache/flink/api/java/sampling/PoissonSampler.java b/flink-java/src/main/java/org/apache/flink/api/java/sampling/PoissonSampler.java index ae8ea57..9b98288 100644 --- a/flink-java/src/main/java/org/apache/flink/api/java/sampling/PoissonSampler.java +++ b/flink-java/src/main/java/org/apache/flink/api/java/sampling/PoissonSampler.java @@ -17,9 +17,9 @@ */ package org.apache.flink.api.java.sampling; -import com.google.common.base.Preconditions; import org.apache.commons.math3.distribution.PoissonDistribution; import org.apache.flink.annotation.Internal; +import org.apache.flink.util.Preconditions; import org.apache.flink.util.XORShiftRandom; import java.util.Iterator; http://git-wip-us.apache.org/repos/asf/flink/blob/15f52112/flink-java/src/main/java/org/apache/flink/api/java/sampling/ReservoirSamplerWithReplacement.java ---------------------------------------------------------------------- diff --git a/flink-java/src/main/java/org/apache/flink/api/java/sampling/ReservoirSamplerWithReplacement.java b/flink-java/src/main/java/org/apache/flink/api/java/sampling/ReservoirSamplerWithReplacement.java index 8a57279..b36d040 100644 --- a/flink-java/src/main/java/org/apache/flink/api/java/sampling/ReservoirSamplerWithReplacement.java +++ b/flink-java/src/main/java/org/apache/flink/api/java/sampling/ReservoirSamplerWithReplacement.java @@ -17,8 +17,8 @@ */ package org.apache.flink.api.java.sampling; -import com.google.common.base.Preconditions; import org.apache.flink.annotation.Internal; +import org.apache.flink.util.Preconditions; import org.apache.flink.util.XORShiftRandom; import java.util.Iterator; http://git-wip-us.apache.org/repos/asf/flink/blob/15f52112/flink-java/src/main/java/org/apache/flink/api/java/sampling/ReservoirSamplerWithoutReplacement.java ---------------------------------------------------------------------- diff --git a/flink-java/src/main/java/org/apache/flink/api/java/sampling/ReservoirSamplerWithoutReplacement.java b/flink-java/src/main/java/org/apache/flink/api/java/sampling/ReservoirSamplerWithoutReplacement.java index 59161ab..a38a921 100644 --- a/flink-java/src/main/java/org/apache/flink/api/java/sampling/ReservoirSamplerWithoutReplacement.java +++ b/flink-java/src/main/java/org/apache/flink/api/java/sampling/ReservoirSamplerWithoutReplacement.java @@ -17,8 +17,8 @@ */ package org.apache.flink.api.java.sampling; -import com.google.common.base.Preconditions; import org.apache.flink.annotation.Internal; +import org.apache.flink.util.Preconditions; import org.apache.flink.util.XORShiftRandom; import java.util.Iterator; http://git-wip-us.apache.org/repos/asf/flink/blob/15f52112/flink-java/src/main/java/org/apache/flink/api/java/utils/DataSetUtils.java ---------------------------------------------------------------------- diff --git a/flink-java/src/main/java/org/apache/flink/api/java/utils/DataSetUtils.java b/flink-java/src/main/java/org/apache/flink/api/java/utils/DataSetUtils.java index 692eea7..1d65c7e 100644 --- a/flink-java/src/main/java/org/apache/flink/api/java/utils/DataSetUtils.java +++ b/flink-java/src/main/java/org/apache/flink/api/java/utils/DataSetUtils.java @@ -18,7 +18,6 @@ package org.apache.flink.api.java.utils; -import com.google.common.collect.Lists; import org.apache.flink.annotation.PublicEvolving; import org.apache.flink.api.common.JobExecutionResult; import org.apache.flink.api.common.distributions.DataDistribution; @@ -48,6 +47,7 @@ import org.apache.flink.configuration.Configuration; import org.apache.flink.util.AbstractID; import org.apache.flink.util.Collector; +import java.util.ArrayList; import java.util.Collections; import java.util.Comparator; import java.util.List; @@ -104,7 +104,10 @@ public final class DataSetUtils { @Override public List<Tuple2<Integer, Long>> initializeBroadcastVariable(Iterable<Tuple2<Integer, Long>> data) { // sort the list by task id to calculate the correct offset - List<Tuple2<Integer, Long>> sortedData = Lists.newArrayList(data); + List<Tuple2<Integer, Long>> sortedData = new ArrayList<>(); + for (Tuple2<Integer, Long> datum : data) { + sortedData.add(datum); + } Collections.sort(sortedData, new Comparator<Tuple2<Integer, Long>>() { @Override public int compare(Tuple2<Integer, Long> o1, Tuple2<Integer, Long> o2) { http://git-wip-us.apache.org/repos/asf/flink/blob/15f52112/flink-java/src/main/java/org/apache/flink/api/java/utils/ParameterTool.java ---------------------------------------------------------------------- diff --git a/flink-java/src/main/java/org/apache/flink/api/java/utils/ParameterTool.java b/flink-java/src/main/java/org/apache/flink/api/java/utils/ParameterTool.java index bfd6d12..6be78e2 100644 --- a/flink-java/src/main/java/org/apache/flink/api/java/utils/ParameterTool.java +++ b/flink-java/src/main/java/org/apache/flink/api/java/utils/ParameterTool.java @@ -17,13 +17,13 @@ */ package org.apache.flink.api.java.utils; -import com.google.common.base.Preconditions; import org.apache.commons.cli.Option; import org.apache.commons.lang3.math.NumberUtils; import org.apache.flink.annotation.PublicEvolving; import org.apache.flink.annotation.Public; import org.apache.flink.api.common.ExecutionConfig; import org.apache.flink.configuration.Configuration; +import org.apache.flink.util.Preconditions; import org.apache.hadoop.util.GenericOptionsParser; import java.io.File; http://git-wip-us.apache.org/repos/asf/flink/blob/15f52112/flink-java/src/test/java/org/apache/flink/api/java/io/CsvInputFormatTest.java ---------------------------------------------------------------------- diff --git a/flink-java/src/test/java/org/apache/flink/api/java/io/CsvInputFormatTest.java b/flink-java/src/test/java/org/apache/flink/api/java/io/CsvInputFormatTest.java index b89f6f5..f44fe9e 100644 --- a/flink-java/src/test/java/org/apache/flink/api/java/io/CsvInputFormatTest.java +++ b/flink-java/src/test/java/org/apache/flink/api/java/io/CsvInputFormatTest.java @@ -19,8 +19,6 @@ package org.apache.flink.api.java.io; -import com.google.common.base.Charsets; - import org.apache.flink.api.common.io.ParseException; import org.apache.flink.api.common.typeutils.CompositeType; import org.apache.flink.api.java.tuple.*; @@ -38,6 +36,7 @@ import java.io.File; import java.io.FileOutputStream; import java.io.IOException; import java.io.OutputStreamWriter; +import java.nio.charset.StandardCharsets; import java.util.ArrayList; import java.util.List; @@ -692,7 +691,7 @@ public class CsvInputFormatTest { tempFile.deleteOnExit(); OutputStreamWriter wrt = new OutputStreamWriter( - new FileOutputStream(tempFile), Charsets.UTF_8 + new FileOutputStream(tempFile), StandardCharsets.UTF_8 ); wrt.write(content); wrt.close(); http://git-wip-us.apache.org/repos/asf/flink/blob/15f52112/flink-java/src/test/java/org/apache/flink/api/java/sampling/RandomSamplerTest.java ---------------------------------------------------------------------- diff --git a/flink-java/src/test/java/org/apache/flink/api/java/sampling/RandomSamplerTest.java b/flink-java/src/test/java/org/apache/flink/api/java/sampling/RandomSamplerTest.java index 197fb2d..68f9154 100644 --- a/flink-java/src/test/java/org/apache/flink/api/java/sampling/RandomSamplerTest.java +++ b/flink-java/src/test/java/org/apache/flink/api/java/sampling/RandomSamplerTest.java @@ -17,16 +17,15 @@ */ package org.apache.flink.api.java.sampling; -import com.google.common.base.Preconditions; import com.google.common.collect.Lists; import com.google.common.collect.Sets; import org.apache.commons.math3.stat.inference.KolmogorovSmirnovTest; +import org.apache.flink.util.Preconditions; import org.junit.BeforeClass; import org.junit.Test; import java.util.ArrayList; import java.util.Collections; -import java.util.Comparator; import java.util.Iterator; import java.util.LinkedList; import java.util.List; http://git-wip-us.apache.org/repos/asf/flink/blob/15f52112/flink-java/src/test/java/org/apache/flink/api/java/tuple/TupleGenerator.java ---------------------------------------------------------------------- diff --git a/flink-java/src/test/java/org/apache/flink/api/java/tuple/TupleGenerator.java b/flink-java/src/test/java/org/apache/flink/api/java/tuple/TupleGenerator.java index 8ac9c59..4420e99 100644 --- a/flink-java/src/test/java/org/apache/flink/api/java/tuple/TupleGenerator.java +++ b/flink-java/src/test/java/org/apache/flink/api/java/tuple/TupleGenerator.java @@ -20,9 +20,9 @@ package org.apache.flink.api.java.tuple; import java.io.File; import java.io.IOException; +import java.nio.charset.StandardCharsets; import java.util.Scanner; -import com.google.common.base.Charsets; import com.google.common.io.Files; /** @@ -95,7 +95,7 @@ class TupleGenerator { } private static void insertCodeIntoFile(String code, File file) throws IOException { - String fileContent = Files.toString(file, Charsets.UTF_8); + String fileContent = Files.toString(file, StandardCharsets.UTF_8); try (Scanner s = new Scanner(fileContent)) { StringBuilder sb = new StringBuilder(); @@ -137,7 +137,7 @@ class TupleGenerator { sb.append(line).append("\n"); } s.close(); - Files.write(sb.toString(), file, Charsets.UTF_8); + Files.write(sb.toString(), file, StandardCharsets.UTF_8); } } @@ -495,4 +495,4 @@ class TupleGenerator { "// THIS IS A GENERATED SOURCE FILE. DO NOT EDIT!\n" + "// GENERATED FROM " + TupleGenerator.class.getName() + ".\n" + "// --------------------------------------------------------------\n\n\n"; -} \ No newline at end of file +}