[FLINK-8762] [quickstarts] Make 'StreamingJob' the default main class and remove WordCount example from the quickstart.
The packaged example jobs have been reported to not be terribly helpful and simply create noise in the initial project setup. Project: http://git-wip-us.apache.org/repos/asf/flink/repo Commit: http://git-wip-us.apache.org/repos/asf/flink/commit/be2ecac4 Tree: http://git-wip-us.apache.org/repos/asf/flink/tree/be2ecac4 Diff: http://git-wip-us.apache.org/repos/asf/flink/diff/be2ecac4 Branch: refs/heads/master Commit: be2ecac40f2689e190a6021d763bf07d94373ba1 Parents: 6fba46e Author: Stephan Ewen <[email protected]> Authored: Wed Feb 21 20:20:51 2018 +0100 Committer: Stephan Ewen <[email protected]> Committed: Mon Feb 26 12:25:10 2018 +0100 ---------------------------------------------------------------------- .../main/resources/archetype-resources/pom.xml | 6 +- .../src/main/java/BatchJob.java | 20 +--- .../main/java/SocketTextStreamWordCount.java | 108 ------------------- .../src/main/java/StreamingJob.java | 21 ++-- .../src/main/java/WordCount.java | 94 ---------------- .../main/resources/archetype-resources/pom.xml | 6 +- .../src/main/scala/BatchJob.scala | 25 ++--- .../main/scala/SocketTextStreamWordCount.scala | 69 ------------ .../src/main/scala/StreamingJob.scala | 24 ++--- .../src/main/scala/WordCount.scala | 53 --------- 10 files changed, 27 insertions(+), 399 deletions(-) ---------------------------------------------------------------------- http://git-wip-us.apache.org/repos/asf/flink/blob/be2ecac4/flink-quickstart/flink-quickstart-java/src/main/resources/archetype-resources/pom.xml ---------------------------------------------------------------------- diff --git a/flink-quickstart/flink-quickstart-java/src/main/resources/archetype-resources/pom.xml b/flink-quickstart/flink-quickstart-java/src/main/resources/archetype-resources/pom.xml index ef900a1..b78bf69 100644 --- a/flink-quickstart/flink-quickstart-java/src/main/resources/archetype-resources/pom.xml +++ b/flink-quickstart/flink-quickstart-java/src/main/resources/archetype-resources/pom.xml @@ -189,15 +189,11 @@ under the License. </excludes> </filter> </filters> - <!-- If you want to use ./bin/flink run <quickstart jar> uncomment the following lines. - This will add a Main-Class entry to the manifest file --> - <!-- <transformers> <transformer implementation="org.apache.maven.plugins.shade.resource.ManifestResourceTransformer"> - <mainClass>${package}.StreamingJob</mainClass> + <mainClass>${package}.StreamingJob</mainClass> </transformer> </transformers> - --> </configuration> </execution> </executions> http://git-wip-us.apache.org/repos/asf/flink/blob/be2ecac4/flink-quickstart/flink-quickstart-java/src/main/resources/archetype-resources/src/main/java/BatchJob.java ---------------------------------------------------------------------- diff --git a/flink-quickstart/flink-quickstart-java/src/main/resources/archetype-resources/src/main/java/BatchJob.java b/flink-quickstart/flink-quickstart-java/src/main/resources/archetype-resources/src/main/java/BatchJob.java index d0e68a4..9711924 100644 --- a/flink-quickstart/flink-quickstart-java/src/main/resources/archetype-resources/src/main/java/BatchJob.java +++ b/flink-quickstart/flink-quickstart-java/src/main/resources/archetype-resources/src/main/java/BatchJob.java @@ -23,22 +23,12 @@ import org.apache.flink.api.java.ExecutionEnvironment; /** * Skeleton for a Flink Batch Job. * - * <p>For a full example of a Flink Batch Job, see the WordCountJob.java file in the - * same package/directory or have a look at the website. + * <p>For a tutorial how to write a Flink batch application, check the + * tutorials and examples on the <a href="http://flink.apache.org/docs/stable/">Flink Website</a>. * - * <p>You can also generate a .jar file that you can submit on your Flink - * cluster. - * Just type - * mvn clean package - * in the projects root directory. - * You will find the jar in - * target/${artifactId}-${version}.jar - * From the CLI you can then run - * ./bin/flink run -c ${package}.BatchJob target/${artifactId}-${version}.jar - * - * <p>For more information on the CLI see: - * - * <p>http://flink.apache.org/docs/latest/apis/cli.html + * <p>To package your appliation into a JAR file for execution, + * change the main class in the POM.xml file to this class (simply search for 'mainClass') + * and run 'mvn clean package' on the command line. */ public class BatchJob { http://git-wip-us.apache.org/repos/asf/flink/blob/be2ecac4/flink-quickstart/flink-quickstart-java/src/main/resources/archetype-resources/src/main/java/SocketTextStreamWordCount.java ---------------------------------------------------------------------- diff --git a/flink-quickstart/flink-quickstart-java/src/main/resources/archetype-resources/src/main/java/SocketTextStreamWordCount.java b/flink-quickstart/flink-quickstart-java/src/main/resources/archetype-resources/src/main/java/SocketTextStreamWordCount.java deleted file mode 100644 index 97df489..0000000 --- a/flink-quickstart/flink-quickstart-java/src/main/resources/archetype-resources/src/main/java/SocketTextStreamWordCount.java +++ /dev/null @@ -1,108 +0,0 @@ -package ${package}; - -/* - * 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. - */ - -import org.apache.flink.api.common.functions.FlatMapFunction; -import org.apache.flink.api.java.tuple.Tuple2; -import org.apache.flink.streaming.api.datastream.DataStream; -import org.apache.flink.streaming.api.environment.StreamExecutionEnvironment; -import org.apache.flink.util.Collector; - -/** - * This example shows an implementation of WordCount with data from a text - * socket. To run the example make sure that the service providing the text data - * is already up and running. - * - * <p>To start an example socket text stream on your local machine run netcat from - * a command line: <code>nc -lk 9999</code>, where the parameter specifies the - * port number. - * - * <p>Usage: - * <code>SocketTextStreamWordCount <hostname> <port></code> - * <br> - * - * <p>This example shows how to: - * <ul> - * <li>use StreamExecutionEnvironment.socketTextStream - * <li>write a simple Flink program - * <li>write and use user-defined functions - * </ul> - * - * @see <a href="www.openbsd.org/cgi-bin/man.cgi?query=nc">netcat</a> - */ -public class SocketTextStreamWordCount { - - // - // Program - // - - public static void main(String[] args) throws Exception { - - if (args.length != 2){ - System.err.println("USAGE:\nSocketTextStreamWordCount <hostname> <port>"); - return; - } - - String hostName = args[0]; - Integer port = Integer.parseInt(args[1]); - - // set up the execution environment - final StreamExecutionEnvironment env = StreamExecutionEnvironment - .getExecutionEnvironment(); - - // get input data - DataStream<String> text = env.socketTextStream(hostName, port); - - DataStream<Tuple2<String, Integer>> counts = - // split up the lines in pairs (2-tuples) containing: (word,1) - text.flatMap(new LineSplitter()) - // group by the tuple field "0" and sum up tuple field "1" - .keyBy(0) - .sum(1); - - counts.print(); - - // execute program - env.execute("Java WordCount from SocketTextStream Example"); - } - - // - // User Functions - // - - /** - * Implements the string tokenizer that splits sentences into words as a user-defined - * FlatMapFunction. The function takes a line (String) and splits it into - * multiple pairs in the form of "(word,1)" (Tuple2<String, Integer>). - */ - public static final class LineSplitter implements FlatMapFunction<String, Tuple2<String, Integer>> { - - @Override - public void flatMap(String value, Collector<Tuple2<String, Integer>> out) { - // normalize and split the line - String[] tokens = value.toLowerCase().split("\\W+"); - - // emit the pairs - for (String token : tokens) { - if (token.length() > 0) { - out.collect(new Tuple2<String, Integer>(token, 1)); - } - } - } - } -} http://git-wip-us.apache.org/repos/asf/flink/blob/be2ecac4/flink-quickstart/flink-quickstart-java/src/main/resources/archetype-resources/src/main/java/StreamingJob.java ---------------------------------------------------------------------- diff --git a/flink-quickstart/flink-quickstart-java/src/main/resources/archetype-resources/src/main/java/StreamingJob.java b/flink-quickstart/flink-quickstart-java/src/main/resources/archetype-resources/src/main/java/StreamingJob.java index 45a67ae..6027e75 100644 --- a/flink-quickstart/flink-quickstart-java/src/main/resources/archetype-resources/src/main/java/StreamingJob.java +++ b/flink-quickstart/flink-quickstart-java/src/main/resources/archetype-resources/src/main/java/StreamingJob.java @@ -20,26 +20,17 @@ package ${package}; import org.apache.flink.streaming.api.environment.StreamExecutionEnvironment; - /** * Skeleton for a Flink Streaming Job. * - * <p>For a full example of a Flink Streaming Job, see the SocketTextStreamWordCount.java - * file in the same package/directory or have a look at the website. - * - * <p>You can also generate a .jar file that you can submit on your Flink - * cluster. - * Just type - * mvn clean package - * in the projects root directory. - * You will find the jar in - * target/${artifactId}-${version}.jar - * From the CLI you can then run - * ./bin/flink run -c ${package}.StreamingJob target/${artifactId}-${version}.jar + * <p>For a tutorial how to write a Flink streaming application, check the + * tutorials and examples on the <a href="http://flink.apache.org/docs/stable/">Flink Website</a>. * - * <p>For more information on the CLI see: + * <p>To package your appliation into a JAR file for execution, run + * 'mvn clean package' on the command line. * - * <p>http://flink.apache.org/docs/latest/apis/cli.html + * <p>If you change the name of the main class (with the public static void main(String[] args)) + * method, change the respective entry in the POM.xml file (simply search for 'mainClass'). */ public class StreamingJob { http://git-wip-us.apache.org/repos/asf/flink/blob/be2ecac4/flink-quickstart/flink-quickstart-java/src/main/resources/archetype-resources/src/main/java/WordCount.java ---------------------------------------------------------------------- diff --git a/flink-quickstart/flink-quickstart-java/src/main/resources/archetype-resources/src/main/java/WordCount.java b/flink-quickstart/flink-quickstart-java/src/main/resources/archetype-resources/src/main/java/WordCount.java deleted file mode 100644 index 6c95389..0000000 --- a/flink-quickstart/flink-quickstart-java/src/main/resources/archetype-resources/src/main/java/WordCount.java +++ /dev/null @@ -1,94 +0,0 @@ -package ${package}; - -/** - * 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. - */ - -import org.apache.flink.api.common.functions.FlatMapFunction; -import org.apache.flink.api.java.DataSet; -import org.apache.flink.api.java.ExecutionEnvironment; -import org.apache.flink.api.java.tuple.Tuple2; -import org.apache.flink.util.Collector; - -/** - * Implements the "WordCount" program that computes a simple word occurrence histogram - * over some sample data - * - * <p>This example shows how to: - * <ul> - * <li>write a simple Flink program. - * <li>use Tuple data types. - * <li>write and use user-defined functions. - * </ul> - * - */ -public class WordCount { - - // - // Program - // - - public static void main(String[] args) throws Exception { - - // set up the execution environment - final ExecutionEnvironment env = ExecutionEnvironment.getExecutionEnvironment(); - - // get input data - DataSet<String> text = env.fromElements( - "To be, or not to be,--that is the question:--", - "Whether 'tis nobler in the mind to suffer", - "The slings and arrows of outrageous fortune", - "Or to take arms against a sea of troubles," - ); - - DataSet<Tuple2<String, Integer>> counts = - // split up the lines in pairs (2-tuples) containing: (word,1) - text.flatMap(new LineSplitter()) - // group by the tuple field "0" and sum up tuple field "1" - .groupBy(0) - .sum(1); - - // execute and print result - counts.print(); - - } - - // - // User Functions - // - - /** - * Implements the string tokenizer that splits sentences into words as a user-defined - * FlatMapFunction. The function takes a line (String) and splits it into - * multiple pairs in the form of "(word,1)" (Tuple2<String, Integer>). - */ - public static final class LineSplitter implements FlatMapFunction<String, Tuple2<String, Integer>> { - - @Override - public void flatMap(String value, Collector<Tuple2<String, Integer>> out) { - // normalize and split the line - String[] tokens = value.toLowerCase().split("\\W+"); - - // emit the pairs - for (String token : tokens) { - if (token.length() > 0) { - out.collect(new Tuple2<String, Integer>(token, 1)); - } - } - } - } -} http://git-wip-us.apache.org/repos/asf/flink/blob/be2ecac4/flink-quickstart/flink-quickstart-scala/src/main/resources/archetype-resources/pom.xml ---------------------------------------------------------------------- diff --git a/flink-quickstart/flink-quickstart-scala/src/main/resources/archetype-resources/pom.xml b/flink-quickstart/flink-quickstart-scala/src/main/resources/archetype-resources/pom.xml index c82c385..2af7118 100644 --- a/flink-quickstart/flink-quickstart-scala/src/main/resources/archetype-resources/pom.xml +++ b/flink-quickstart/flink-quickstart-scala/src/main/resources/archetype-resources/pom.xml @@ -201,15 +201,11 @@ under the License. </excludes> </filter> </filters> - <!-- If you want to use ./bin/flink run <quickstart jar> uncomment the following lines. - This will add a Main-Class entry to the manifest file --> - <!-- <transformers> <transformer implementation="org.apache.maven.plugins.shade.resource.ManifestResourceTransformer"> - <mainClass>${package}.StreamingJob</mainClass> + <mainClass>${package}.StreamingJob</mainClass> </transformer> </transformers> - --> </configuration> </execution> </executions> http://git-wip-us.apache.org/repos/asf/flink/blob/be2ecac4/flink-quickstart/flink-quickstart-scala/src/main/resources/archetype-resources/src/main/scala/BatchJob.scala ---------------------------------------------------------------------- diff --git a/flink-quickstart/flink-quickstart-scala/src/main/resources/archetype-resources/src/main/scala/BatchJob.scala b/flink-quickstart/flink-quickstart-scala/src/main/resources/archetype-resources/src/main/scala/BatchJob.scala index 4ecfeed..a533da9 100644 --- a/flink-quickstart/flink-quickstart-scala/src/main/resources/archetype-resources/src/main/scala/BatchJob.scala +++ b/flink-quickstart/flink-quickstart-scala/src/main/resources/archetype-resources/src/main/scala/BatchJob.scala @@ -23,26 +23,15 @@ import org.apache.flink.api.scala._ /** * Skeleton for a Flink Batch Job. * - * For a full example of a Flink Batch Job, see the WordCountJob.scala file in the - * same package/directory or have a look at the website. + * For a tutorial how to write a Flink batch application, check the + * tutorials and examples on the <a href="http://flink.apache.org/docs/stable/">Flink Website</a>. * - * You can also generate a .jar file that you can submit on your Flink - * cluster. Just type - * {{{ - * mvn clean package - * }}} - * in the projects root directory. You will find the jar in - * target/${artifactId}-${version}.jar - * From the CLI you can then run - * {{{ - * ./bin/flink run -c ${package}.BatchJob target/${artifactId}-${version}.jar - * }}} - * - * For more information on the CLI see: - * - * http://flink.apache.org/docs/latest/apis/cli.html + * To package your appliation into a JAR file for execution, + * change the main class in the POM.xml file to this class (simply search for 'mainClass') + * and run 'mvn clean package' on the command line. */ object BatchJob { + def main(args: Array[String]) { // set up the batch execution environment val env = ExecutionEnvironment.getExecutionEnvironment @@ -74,4 +63,4 @@ object BatchJob { // execute program env.execute("Flink Batch Scala API Skeleton") } -} \ No newline at end of file +} http://git-wip-us.apache.org/repos/asf/flink/blob/be2ecac4/flink-quickstart/flink-quickstart-scala/src/main/resources/archetype-resources/src/main/scala/SocketTextStreamWordCount.scala ---------------------------------------------------------------------- diff --git a/flink-quickstart/flink-quickstart-scala/src/main/resources/archetype-resources/src/main/scala/SocketTextStreamWordCount.scala b/flink-quickstart/flink-quickstart-scala/src/main/resources/archetype-resources/src/main/scala/SocketTextStreamWordCount.scala deleted file mode 100644 index a6987ac..0000000 --- a/flink-quickstart/flink-quickstart-scala/src/main/resources/archetype-resources/src/main/scala/SocketTextStreamWordCount.scala +++ /dev/null @@ -1,69 +0,0 @@ -package ${package} - -/* - * 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. - */ - -import org.apache.flink.streaming.api.scala._ - -/** - * This example shows an implementation of WordCount with data from a text socket. - * To run the example make sure that the service providing the text data is already up and running. - * - * To start an example socket text stream on your local machine run netcat from a command line, - * where the parameter specifies the port number: - * - * {{{ - * nc -lk 9999 - * }}} - * - * Usage: - * {{{ - * SocketTextStreamWordCount <hostname> <port> - * }}} - * - * This example shows how to: - * - * - use StreamExecutionEnvironment.socketTextStream - * - write a simple Flink Streaming program in scala - * - write and use user-defined functions - */ -object SocketTextStreamWordCount { - - def main(args: Array[String]) { - if (args.length != 2) { - System.err.println("USAGE:\nSocketTextStreamWordCount <hostname> <port>") - return - } - - val hostName = args(0) - val port = args(1).toInt - - val env = StreamExecutionEnvironment.getExecutionEnvironment - - // create streams for names and ages by mapping the inputs to the corresponding objects - val text = env.socketTextStream(hostName, port) - val counts = text.flatMap { _.toLowerCase.split("\\W+") filter { _.nonEmpty } } - .map { (_, 1) } - .keyBy(0) - .sum(1) - - counts print - - env.execute("Scala WordCount from SocketTextStream Example") - } -} http://git-wip-us.apache.org/repos/asf/flink/blob/be2ecac4/flink-quickstart/flink-quickstart-scala/src/main/resources/archetype-resources/src/main/scala/StreamingJob.scala ---------------------------------------------------------------------- diff --git a/flink-quickstart/flink-quickstart-scala/src/main/resources/archetype-resources/src/main/scala/StreamingJob.scala b/flink-quickstart/flink-quickstart-scala/src/main/resources/archetype-resources/src/main/scala/StreamingJob.scala index 7a45fc2..7c950b1 100644 --- a/flink-quickstart/flink-quickstart-scala/src/main/resources/archetype-resources/src/main/scala/StreamingJob.scala +++ b/flink-quickstart/flink-quickstart-scala/src/main/resources/archetype-resources/src/main/scala/StreamingJob.scala @@ -23,24 +23,14 @@ import org.apache.flink.streaming.api.scala._ /** * Skeleton for a Flink Streaming Job. * - * For a full example of a Flink Streaming Job, see the SocketTextStreamWordCount.java - * file in the same package/directory or have a look at the website. + * For a tutorial how to write a Flink streaming application, check the + * tutorials and examples on the <a href="http://flink.apache.org/docs/stable/">Flink Website</a>. * - * You can also generate a .jar file that you can submit on your Flink - * cluster. Just type - * {{{ - * mvn clean package - * }}} - * in the projects root directory. You will find the jar in - * target/${artifactId}-${version}.jar - * From the CLI you can then run - * {{{ - * ./bin/flink run -c ${package}.StreamingJob target/${artifactId}-${version}.jar - * }}} + * To package your appliation into a JAR file for execution, run + * 'mvn clean package' on the command line. * - * For more information on the CLI see: - * - * http://flink.apache.org/docs/latest/apis/cli.html + * If you change the name of the main class (with the public static void main(String[] args)) + * method, change the respective entry in the POM.xml file (simply search for 'mainClass'). */ object StreamingJob { def main(args: Array[String]) { @@ -70,4 +60,4 @@ object StreamingJob { // execute program env.execute("Flink Streaming Scala API Skeleton") } -} \ No newline at end of file +} http://git-wip-us.apache.org/repos/asf/flink/blob/be2ecac4/flink-quickstart/flink-quickstart-scala/src/main/resources/archetype-resources/src/main/scala/WordCount.scala ---------------------------------------------------------------------- diff --git a/flink-quickstart/flink-quickstart-scala/src/main/resources/archetype-resources/src/main/scala/WordCount.scala b/flink-quickstart/flink-quickstart-scala/src/main/resources/archetype-resources/src/main/scala/WordCount.scala deleted file mode 100644 index b88dcc6..0000000 --- a/flink-quickstart/flink-quickstart-scala/src/main/resources/archetype-resources/src/main/scala/WordCount.scala +++ /dev/null @@ -1,53 +0,0 @@ -package ${package} - -/** - * 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. - */ - -import org.apache.flink.api.scala._ - -/** - * Implements the "WordCount" program that computes a simple word occurrence histogram - * over some sample data - * - * This example shows how to: - * - * - write a simple Flink program. - * - use Tuple data types. - * - write and use user-defined functions. - */ -object WordCount { - def main(args: Array[String]) { - - // set up the execution environment - val env = ExecutionEnvironment.getExecutionEnvironment - - // get input data - val text = env.fromElements("To be, or not to be,--that is the question:--", - "Whether 'tis nobler in the mind to suffer", "The slings and arrows of outrageous fortune", - "Or to take arms against a sea of troubles,") - - val counts = text.flatMap { _.toLowerCase.split("\\W+") } - .map { (_, 1) } - .groupBy(0) - .sum(1) - - // execute and print result - counts.print() - - } -}
