diff options
author | Sean Owen <sowen@cloudera.com> | 2016-01-08 17:47:44 +0000 |
---|---|---|
committer | Sean Owen <sowen@cloudera.com> | 2016-01-08 17:47:44 +0000 |
commit | b9c835337880f57fe8b953962913bcc524162348 (patch) | |
tree | 5dc476b1a65d513210d1124db144aaa3c5f66679 /examples/src/main/java | |
parent | 794ea553bd0fcfece15b610b47ee86d6644134c9 (diff) | |
download | spark-b9c835337880f57fe8b953962913bcc524162348.tar.gz spark-b9c835337880f57fe8b953962913bcc524162348.tar.bz2 spark-b9c835337880f57fe8b953962913bcc524162348.zip |
[SPARK-12618][CORE][STREAMING][SQL] Clean up build warnings: 2.0.0 edition
Fix most build warnings: mostly deprecated API usages. I'll annotate some of the changes below. CC rxin who is leading the charge to remove the deprecated APIs.
Author: Sean Owen <sowen@cloudera.com>
Closes #10570 from srowen/SPARK-12618.
Diffstat (limited to 'examples/src/main/java')
5 files changed, 41 insertions, 37 deletions
diff --git a/examples/src/main/java/org/apache/spark/examples/mllib/JavaBinaryClassificationMetricsExample.java b/examples/src/main/java/org/apache/spark/examples/mllib/JavaBinaryClassificationMetricsExample.java index 779fac01c4..3d8babba04 100644 --- a/examples/src/main/java/org/apache/spark/examples/mllib/JavaBinaryClassificationMetricsExample.java +++ b/examples/src/main/java/org/apache/spark/examples/mllib/JavaBinaryClassificationMetricsExample.java @@ -56,6 +56,7 @@ public class JavaBinaryClassificationMetricsExample { // Compute raw scores on the test set. JavaRDD<Tuple2<Object, Object>> predictionAndLabels = test.map( new Function<LabeledPoint, Tuple2<Object, Object>>() { + @Override public Tuple2<Object, Object> call(LabeledPoint p) { Double prediction = model.predict(p.features()); return new Tuple2<Object, Object>(prediction, p.label()); @@ -88,6 +89,7 @@ public class JavaBinaryClassificationMetricsExample { // Thresholds JavaRDD<Double> thresholds = precision.map( new Function<Tuple2<Object, Object>, Double>() { + @Override public Double call(Tuple2<Object, Object> t) { return new Double(t._1().toString()); } @@ -106,8 +108,7 @@ public class JavaBinaryClassificationMetricsExample { // Save and load model model.save(sc, "target/tmp/LogisticRegressionModel"); - LogisticRegressionModel sameModel = LogisticRegressionModel.load(sc, - "target/tmp/LogisticRegressionModel"); + LogisticRegressionModel.load(sc, "target/tmp/LogisticRegressionModel"); // $example off$ } } diff --git a/examples/src/main/java/org/apache/spark/examples/mllib/JavaRankingMetricsExample.java b/examples/src/main/java/org/apache/spark/examples/mllib/JavaRankingMetricsExample.java index 47ab3fc358..4ad2104763 100644 --- a/examples/src/main/java/org/apache/spark/examples/mllib/JavaRankingMetricsExample.java +++ b/examples/src/main/java/org/apache/spark/examples/mllib/JavaRankingMetricsExample.java @@ -41,6 +41,7 @@ public class JavaRankingMetricsExample { JavaRDD<String> data = sc.textFile(path); JavaRDD<Rating> ratings = data.map( new Function<String, Rating>() { + @Override public Rating call(String line) { String[] parts = line.split("::"); return new Rating(Integer.parseInt(parts[0]), Integer.parseInt(parts[1]), Double @@ -57,13 +58,14 @@ public class JavaRankingMetricsExample { JavaRDD<Tuple2<Object, Rating[]>> userRecs = model.recommendProductsForUsers(10).toJavaRDD(); JavaRDD<Tuple2<Object, Rating[]>> userRecsScaled = userRecs.map( new Function<Tuple2<Object, Rating[]>, Tuple2<Object, Rating[]>>() { + @Override public Tuple2<Object, Rating[]> call(Tuple2<Object, Rating[]> t) { Rating[] scaledRatings = new Rating[t._2().length]; for (int i = 0; i < scaledRatings.length; i++) { double newRating = Math.max(Math.min(t._2()[i].rating(), 1.0), 0.0); scaledRatings[i] = new Rating(t._2()[i].user(), t._2()[i].product(), newRating); } - return new Tuple2<Object, Rating[]>(t._1(), scaledRatings); + return new Tuple2<>(t._1(), scaledRatings); } } ); @@ -72,6 +74,7 @@ public class JavaRankingMetricsExample { // Map ratings to 1 or 0, 1 indicating a movie that should be recommended JavaRDD<Rating> binarizedRatings = ratings.map( new Function<Rating, Rating>() { + @Override public Rating call(Rating r) { double binaryRating; if (r.rating() > 0.0) { @@ -87,6 +90,7 @@ public class JavaRankingMetricsExample { // Group ratings by common user JavaPairRDD<Object, Iterable<Rating>> userMovies = binarizedRatings.groupBy( new Function<Rating, Object>() { + @Override public Object call(Rating r) { return r.user(); } @@ -96,8 +100,9 @@ public class JavaRankingMetricsExample { // Get true relevant documents from all user ratings JavaPairRDD<Object, List<Integer>> userMoviesList = userMovies.mapValues( new Function<Iterable<Rating>, List<Integer>>() { + @Override public List<Integer> call(Iterable<Rating> docs) { - List<Integer> products = new ArrayList<Integer>(); + List<Integer> products = new ArrayList<>(); for (Rating r : docs) { if (r.rating() > 0.0) { products.add(r.product()); @@ -111,8 +116,9 @@ public class JavaRankingMetricsExample { // Extract the product id from each recommendation JavaPairRDD<Object, List<Integer>> userRecommendedList = userRecommended.mapValues( new Function<Rating[], List<Integer>>() { + @Override public List<Integer> call(Rating[] docs) { - List<Integer> products = new ArrayList<Integer>(); + List<Integer> products = new ArrayList<>(); for (Rating r : docs) { products.add(r.product()); } @@ -124,7 +130,7 @@ public class JavaRankingMetricsExample { userRecommendedList).values(); // Instantiate the metrics object - RankingMetrics metrics = RankingMetrics.of(relevantDocs); + RankingMetrics<Integer> metrics = RankingMetrics.of(relevantDocs); // Precision and NDCG at k Integer[] kVector = {1, 3, 5}; @@ -139,6 +145,7 @@ public class JavaRankingMetricsExample { // Evaluate the model using numerical ratings and regression metrics JavaRDD<Tuple2<Object, Object>> userProducts = ratings.map( new Function<Rating, Tuple2<Object, Object>>() { + @Override public Tuple2<Object, Object> call(Rating r) { return new Tuple2<Object, Object>(r.user(), r.product()); } @@ -147,18 +154,20 @@ public class JavaRankingMetricsExample { JavaPairRDD<Tuple2<Integer, Integer>, Object> predictions = JavaPairRDD.fromJavaRDD( model.predict(JavaRDD.toRDD(userProducts)).toJavaRDD().map( new Function<Rating, Tuple2<Tuple2<Integer, Integer>, Object>>() { + @Override public Tuple2<Tuple2<Integer, Integer>, Object> call(Rating r) { return new Tuple2<Tuple2<Integer, Integer>, Object>( - new Tuple2<Integer, Integer>(r.user(), r.product()), r.rating()); + new Tuple2<>(r.user(), r.product()), r.rating()); } } )); JavaRDD<Tuple2<Object, Object>> ratesAndPreds = JavaPairRDD.fromJavaRDD(ratings.map( new Function<Rating, Tuple2<Tuple2<Integer, Integer>, Object>>() { + @Override public Tuple2<Tuple2<Integer, Integer>, Object> call(Rating r) { return new Tuple2<Tuple2<Integer, Integer>, Object>( - new Tuple2<Integer, Integer>(r.user(), r.product()), r.rating()); + new Tuple2<>(r.user(), r.product()), r.rating()); } } )).join(predictions).values(); diff --git a/examples/src/main/java/org/apache/spark/examples/streaming/JavaRecoverableNetworkWordCount.java b/examples/src/main/java/org/apache/spark/examples/streaming/JavaRecoverableNetworkWordCount.java index 90d473703e..bc963a02be 100644 --- a/examples/src/main/java/org/apache/spark/examples/streaming/JavaRecoverableNetworkWordCount.java +++ b/examples/src/main/java/org/apache/spark/examples/streaming/JavaRecoverableNetworkWordCount.java @@ -36,6 +36,7 @@ import org.apache.spark.api.java.function.FlatMapFunction; import org.apache.spark.api.java.function.Function; import org.apache.spark.api.java.function.Function2; import org.apache.spark.api.java.function.PairFunction; +import org.apache.spark.api.java.function.VoidFunction2; import org.apache.spark.broadcast.Broadcast; import org.apache.spark.streaming.Durations; import org.apache.spark.streaming.Time; @@ -154,9 +155,9 @@ public final class JavaRecoverableNetworkWordCount { } }); - wordCounts.foreachRDD(new Function2<JavaPairRDD<String, Integer>, Time, Void>() { + wordCounts.foreachRDD(new VoidFunction2<JavaPairRDD<String, Integer>, Time>() { @Override - public Void call(JavaPairRDD<String, Integer> rdd, Time time) throws IOException { + public void call(JavaPairRDD<String, Integer> rdd, Time time) throws IOException { // Get or register the blacklist Broadcast final Broadcast<List<String>> blacklist = JavaWordBlacklist.getInstance(new JavaSparkContext(rdd.context())); // Get or register the droppedWordsCounter Accumulator @@ -164,7 +165,7 @@ public final class JavaRecoverableNetworkWordCount { // Use blacklist to drop words and use droppedWordsCounter to count them String counts = rdd.filter(new Function<Tuple2<String, Integer>, Boolean>() { @Override - public Boolean call(Tuple2<String, Integer> wordCount) throws Exception { + public Boolean call(Tuple2<String, Integer> wordCount) { if (blacklist.value().contains(wordCount._1())) { droppedWordsCounter.add(wordCount._2()); return false; @@ -178,7 +179,6 @@ public final class JavaRecoverableNetworkWordCount { System.out.println("Dropped " + droppedWordsCounter.value() + " word(s) totally"); System.out.println("Appending to " + outputFile.getAbsolutePath()); Files.append(output + "\n", outputFile, Charset.defaultCharset()); - return null; } }); diff --git a/examples/src/main/java/org/apache/spark/examples/streaming/JavaSqlNetworkWordCount.java b/examples/src/main/java/org/apache/spark/examples/streaming/JavaSqlNetworkWordCount.java index 3515d7be45..084f68a8be 100644 --- a/examples/src/main/java/org/apache/spark/examples/streaming/JavaSqlNetworkWordCount.java +++ b/examples/src/main/java/org/apache/spark/examples/streaming/JavaSqlNetworkWordCount.java @@ -26,7 +26,7 @@ import org.apache.spark.SparkContext; import org.apache.spark.api.java.JavaRDD; import org.apache.spark.api.java.function.FlatMapFunction; import org.apache.spark.api.java.function.Function; -import org.apache.spark.api.java.function.Function2; +import org.apache.spark.api.java.function.VoidFunction2; import org.apache.spark.sql.SQLContext; import org.apache.spark.sql.DataFrame; import org.apache.spark.api.java.StorageLevels; @@ -78,13 +78,14 @@ public final class JavaSqlNetworkWordCount { }); // Convert RDDs of the words DStream to DataFrame and run SQL query - words.foreachRDD(new Function2<JavaRDD<String>, Time, Void>() { + words.foreachRDD(new VoidFunction2<JavaRDD<String>, Time>() { @Override - public Void call(JavaRDD<String> rdd, Time time) { + public void call(JavaRDD<String> rdd, Time time) { SQLContext sqlContext = JavaSQLContextSingleton.getInstance(rdd.context()); // Convert JavaRDD[String] to JavaRDD[bean class] to DataFrame JavaRDD<JavaRecord> rowRDD = rdd.map(new Function<String, JavaRecord>() { + @Override public JavaRecord call(String word) { JavaRecord record = new JavaRecord(); record.setWord(word); @@ -101,7 +102,6 @@ public final class JavaSqlNetworkWordCount { sqlContext.sql("select word, count(*) as total from words group by word"); System.out.println("========= " + time + "========="); wordCountsDataFrame.show(); - return null; } }); diff --git a/examples/src/main/java/org/apache/spark/examples/streaming/JavaTwitterHashTagJoinSentiments.java b/examples/src/main/java/org/apache/spark/examples/streaming/JavaTwitterHashTagJoinSentiments.java index 030ee30b93..d869768026 100644 --- a/examples/src/main/java/org/apache/spark/examples/streaming/JavaTwitterHashTagJoinSentiments.java +++ b/examples/src/main/java/org/apache/spark/examples/streaming/JavaTwitterHashTagJoinSentiments.java @@ -17,13 +17,13 @@ package org.apache.spark.examples.streaming; -import org.apache.commons.io.IOUtils; import org.apache.spark.SparkConf; import org.apache.spark.api.java.JavaPairRDD; import org.apache.spark.api.java.function.FlatMapFunction; import org.apache.spark.api.java.function.Function; import org.apache.spark.api.java.function.Function2; import org.apache.spark.api.java.function.PairFunction; +import org.apache.spark.api.java.function.VoidFunction; import org.apache.spark.streaming.Duration; import org.apache.spark.streaming.api.java.JavaDStream; import org.apache.spark.streaming.api.java.JavaPairDStream; @@ -33,8 +33,6 @@ import org.apache.spark.streaming.twitter.TwitterUtils; import scala.Tuple2; import twitter4j.Status; -import java.io.IOException; -import java.net.URI; import java.util.Arrays; import java.util.List; @@ -44,7 +42,7 @@ import java.util.List; */ public class JavaTwitterHashTagJoinSentiments { - public static void main(String[] args) throws IOException { + public static void main(String[] args) { if (args.length < 4) { System.err.println("Usage: JavaTwitterHashTagJoinSentiments <consumer key> <consumer secret>" + " <access token> <access token secret> [<filters>]"); @@ -79,7 +77,7 @@ public class JavaTwitterHashTagJoinSentiments { JavaDStream<String> hashTags = words.filter(new Function<String, Boolean>() { @Override - public Boolean call(String word) throws Exception { + public Boolean call(String word) { return word.startsWith("#"); } }); @@ -91,8 +89,7 @@ public class JavaTwitterHashTagJoinSentiments { @Override public Tuple2<String, Double> call(String line) { String[] columns = line.split("\t"); - return new Tuple2<String, Double>(columns[0], - Double.parseDouble(columns[1])); + return new Tuple2<>(columns[0], Double.parseDouble(columns[1])); } }); @@ -101,7 +98,7 @@ public class JavaTwitterHashTagJoinSentiments { @Override public Tuple2<String, Integer> call(String s) { // leave out the # character - return new Tuple2<String, Integer>(s.substring(1), 1); + return new Tuple2<>(s.substring(1), 1); } }); @@ -120,9 +117,8 @@ public class JavaTwitterHashTagJoinSentiments { hashTagTotals.transformToPair(new Function<JavaPairRDD<String, Integer>, JavaPairRDD<String, Tuple2<Double, Integer>>>() { @Override - public JavaPairRDD<String, Tuple2<Double, Integer>> call(JavaPairRDD<String, - Integer> topicCount) - throws Exception { + public JavaPairRDD<String, Tuple2<Double, Integer>> call( + JavaPairRDD<String, Integer> topicCount) { return wordSentiments.join(topicCount); } }); @@ -131,9 +127,9 @@ public class JavaTwitterHashTagJoinSentiments { new PairFunction<Tuple2<String, Tuple2<Double, Integer>>, String, Double>() { @Override public Tuple2<String, Double> call(Tuple2<String, - Tuple2<Double, Integer>> topicAndTuplePair) throws Exception { + Tuple2<Double, Integer>> topicAndTuplePair) { Tuple2<Double, Integer> happinessAndCount = topicAndTuplePair._2(); - return new Tuple2<String, Double>(topicAndTuplePair._1(), + return new Tuple2<>(topicAndTuplePair._1(), happinessAndCount._1() * happinessAndCount._2()); } }); @@ -141,9 +137,8 @@ public class JavaTwitterHashTagJoinSentiments { JavaPairDStream<Double, String> happinessTopicPairs = topicHappiness.mapToPair( new PairFunction<Tuple2<String, Double>, Double, String>() { @Override - public Tuple2<Double, String> call(Tuple2<String, Double> topicHappiness) - throws Exception { - return new Tuple2<Double, String>(topicHappiness._2(), + public Tuple2<Double, String> call(Tuple2<String, Double> topicHappiness) { + return new Tuple2<>(topicHappiness._2(), topicHappiness._1()); } }); @@ -151,17 +146,17 @@ public class JavaTwitterHashTagJoinSentiments { JavaPairDStream<Double, String> happiest10 = happinessTopicPairs.transformToPair( new Function<JavaPairRDD<Double, String>, JavaPairRDD<Double, String>>() { @Override - public JavaPairRDD<Double, String> call(JavaPairRDD<Double, - String> happinessAndTopics) throws Exception { + public JavaPairRDD<Double, String> call( + JavaPairRDD<Double, String> happinessAndTopics) { return happinessAndTopics.sortByKey(false); } } ); // Print hash tags with the most positive sentiment values - happiest10.foreachRDD(new Function<JavaPairRDD<Double, String>, Void>() { + happiest10.foreachRDD(new VoidFunction<JavaPairRDD<Double, String>>() { @Override - public Void call(JavaPairRDD<Double, String> happinessTopicPairs) throws Exception { + public void call(JavaPairRDD<Double, String> happinessTopicPairs) { List<Tuple2<Double, String>> topList = happinessTopicPairs.take(10); System.out.println( String.format("\nHappiest topics in last 10 seconds (%s total):", @@ -170,7 +165,6 @@ public class JavaTwitterHashTagJoinSentiments { System.out.println( String.format("%s (%s happiness)", pair._2(), pair._1())); } - return null; } }); |