[2/3] incubator-beam git commit: [BEAM-716] Fix javadoc on with* methods [BEAM-959] Improve check preconditions in JmsIO
[BEAM-716] Fix javadoc on with* methods [BEAM-959] Improve check preconditions in JmsIO Project: http://git-wip-us.apache.org/repos/asf/incubator-beam/repo Commit: http://git-wip-us.apache.org/repos/asf/incubator-beam/commit/30e14cfa Tree: http://git-wip-us.apache.org/repos/asf/incubator-beam/tree/30e14cfa Diff: http://git-wip-us.apache.org/repos/asf/incubator-beam/diff/30e14cfa Branch: refs/heads/master Commit: 30e14cfa63db50d567185599ea049c96229b48e2 Parents: caf1c72 Author: Jean-Baptiste OnofréAuthored: Tue Dec 13 21:55:46 2016 +0100 Committer: Jean-Baptiste Onofré Committed: Mon Dec 19 07:24:05 2016 +0100 -- .../java/org/apache/beam/sdk/io/jms/JmsIO.java | 45 +--- 1 file changed, 30 insertions(+), 15 deletions(-) -- http://git-wip-us.apache.org/repos/asf/incubator-beam/blob/30e14cfa/sdks/java/io/jms/src/main/java/org/apache/beam/sdk/io/jms/JmsIO.java -- diff --git a/sdks/java/io/jms/src/main/java/org/apache/beam/sdk/io/jms/JmsIO.java b/sdks/java/io/jms/src/main/java/org/apache/beam/sdk/io/jms/JmsIO.java index 76dee67..b6de26a 100644 --- a/sdks/java/io/jms/src/main/java/org/apache/beam/sdk/io/jms/JmsIO.java +++ b/sdks/java/io/jms/src/main/java/org/apache/beam/sdk/io/jms/JmsIO.java @@ -18,7 +18,7 @@ package org.apache.beam.sdk.io.jms; import static com.google.common.base.Preconditions.checkArgument; -import static com.google.common.base.Preconditions.checkNotNull; +import static com.google.common.base.Preconditions.checkState; import com.google.auto.value.AutoValue; import com.google.common.annotations.VisibleForTesting; @@ -145,7 +145,7 @@ public class JmsIO { } /** - * Specify the JMS connection factory to connect to the JMS broker. + * Specify the JMS connection factory to connect to the JMS broker. * * For instance: * @@ -159,11 +159,13 @@ public class JmsIO { * @return The corresponding {@link JmsIO.Read}. */ public Read withConnectionFactory(ConnectionFactory connectionFactory) { + checkArgument(connectionFactory != null, "withConnectionFactory(connectionFactory) called" + + " with null connectionFactory"); return builder().setConnectionFactory(connectionFactory).build(); } /** - * Specify the JMS queue destination name where to read messages from. The + * Specify the JMS queue destination name where to read messages from. The * {@link JmsIO.Read} acts as a consumer on the queue. * * This method is exclusive with {@link JmsIO.Read#withTopic(String)}. The user has to @@ -181,11 +183,12 @@ public class JmsIO { * @return The corresponding {@link JmsIO.Read}. */ public Read withQueue(String queue) { + checkArgument(queue != null, "withQueue(queue) called with null queue"); return builder().setQueue(queue).build(); } /** - * Specify the JMS topic destination name where to receive messages from. The + * Specify the JMS topic destination name where to receive messages from. The * {@link JmsIO.Read} acts as a subscriber on the topic. * * This method is exclusive with {@link JmsIO.Read#withQueue(String)}. The user has to @@ -203,11 +206,12 @@ public class JmsIO { * @return The corresponding {@link JmsIO.Read}. */ public Read withTopic(String topic) { + checkArgument(topic != null, "withTopic(topic) called with null topic"); return builder().setTopic(topic).build(); } /** - * Define the max number of records that the source will read. Using a max number of records + * Define the max number of records that the source will read. Using a max number of records * different from {@code Long.MAX_VALUE} means the source will be {@code Bounded}, and will * stop once the max number of records read is reached. * @@ -223,11 +227,13 @@ public class JmsIO { * @return The corresponding {@link JmsIO.Read}. */ public Read withMaxNumRecords(long maxNumRecords) { + checkArgument(maxNumRecords >= 0, "withMaxNumRecords(maxNumRecords) called with invalid " + + "maxNumRecords"); return builder().setMaxNumRecords(maxNumRecords).build(); } /** - * Define the max read time that the source will read. Using a non null max read time + * Define the max read time that the source will read. Using a non null max read time * duration means the source will be {@code Bounded}, and will stop once the max read time is * reached. * @@ -243,6 +249,8 @@ public class JmsIO { * @return The corresponding {@link JmsIO.Read}. */ public Read withMaxReadTime(Duration maxReadTime) { + checkArgument(maxReadTime != null,
[1/3] incubator-beam git commit: [BEAM-716] Use AutoValue in JmsIO
Repository: incubator-beam Updated Branches: refs/heads/master 1c9bf8d66 -> 1e148cd7d [BEAM-716] Use AutoValue in JmsIO Project: http://git-wip-us.apache.org/repos/asf/incubator-beam/repo Commit: http://git-wip-us.apache.org/repos/asf/incubator-beam/commit/caf1c720 Tree: http://git-wip-us.apache.org/repos/asf/incubator-beam/tree/caf1c720 Diff: http://git-wip-us.apache.org/repos/asf/incubator-beam/diff/caf1c720 Branch: refs/heads/master Commit: caf1c720f66de4d502f79b6c11c64b49c53329b0 Parents: 1c9bf8d Author: Jean-Baptiste OnofréAuthored: Sun Dec 11 07:43:41 2016 +0100 Committer: Jean-Baptiste Onofré Committed: Mon Dec 19 07:24:00 2016 +0100 -- sdks/java/io/jms/pom.xml| 7 + .../java/org/apache/beam/sdk/io/jms/JmsIO.java | 321 +-- 2 files changed, 228 insertions(+), 100 deletions(-) -- http://git-wip-us.apache.org/repos/asf/incubator-beam/blob/caf1c720/sdks/java/io/jms/pom.xml -- diff --git a/sdks/java/io/jms/pom.xml b/sdks/java/io/jms/pom.xml index bca0152..b88254e 100644 --- a/sdks/java/io/jms/pom.xml +++ b/sdks/java/io/jms/pom.xml @@ -81,6 +81,13 @@ jsr305 + + + com.google.auto.value + auto-value + provided + + org.apache.activemq http://git-wip-us.apache.org/repos/asf/incubator-beam/blob/caf1c720/sdks/java/io/jms/src/main/java/org/apache/beam/sdk/io/jms/JmsIO.java -- diff --git a/sdks/java/io/jms/src/main/java/org/apache/beam/sdk/io/jms/JmsIO.java b/sdks/java/io/jms/src/main/java/org/apache/beam/sdk/io/jms/JmsIO.java index 24fa67d..76dee67 100644 --- a/sdks/java/io/jms/src/main/java/org/apache/beam/sdk/io/jms/JmsIO.java +++ b/sdks/java/io/jms/src/main/java/org/apache/beam/sdk/io/jms/JmsIO.java @@ -20,6 +20,7 @@ package org.apache.beam.sdk.io.jms; import static com.google.common.base.Preconditions.checkArgument; import static com.google.common.base.Preconditions.checkNotNull; +import com.google.auto.value.AutoValue; import com.google.common.annotations.VisibleForTesting; import java.io.IOException; import java.util.ArrayList; @@ -101,37 +102,148 @@ public class JmsIO { private static final Logger LOG = LoggerFactory.getLogger(JmsIO.class); public static Read read() { -return new Read(null, null, null, Long.MAX_VALUE, null); +return new AutoValue_JmsIO_Read.Builder().setMaxNumRecords(Long.MAX_VALUE).build(); } public static Write write() { -return new Write(null, null, null); +return new AutoValue_JmsIO_Write.Builder().build(); } /** * A {@link PTransform} to read from a JMS destination. See {@link JmsIO} for more * information on usage and configuration. */ - public static class Read extends PTransform { + @AutoValue + public abstract static class Read extends PTransform { +/** + * NB: According to http://docs.oracle.com/javaee/1.4/api/javax/jms/ConnectionFactory.html + * "It is expected that JMS providers will provide the tools an administrator needs to create + * and configure administered objects in a JNDI namespace. JMS provider implementations of + * administered objects should be both javax.jndi.Referenceable and java.io.Serializable so + * that they can be stored in all JNDI naming contexts. In addition, it is recommended that + * these implementations follow the JavaBeansTM design patterns." + * + * So, a {@link ConnectionFactory} implementation is serializable. + */ +@Nullable abstract ConnectionFactory getConnectionFactory(); +@Nullable abstract String getQueue(); +@Nullable abstract String getTopic(); +abstract long getMaxNumRecords(); +@Nullable abstract Duration getMaxReadTime(); + +abstract Builder builder(); + +@AutoValue.Builder +abstract static class Builder { + abstract Builder setConnectionFactory(ConnectionFactory connectionFactory); + abstract Builder setQueue(String queue); + abstract Builder setTopic(String topic); + abstract Builder setMaxNumRecords(long maxNumRecords); + abstract Builder setMaxReadTime(Duration maxReadTime); + abstract Read build(); +} + +/** + * Specify the JMS connection factory to connect to the JMS broker. + * + * For instance: + * + * + * {@code + * pipeline.apply(JmsIO.read().withConnectionFactory(myConnectionFactory) + * } + * + * + * @param connectionFactory The JMS {@link ConnectionFactory}. + * @return The corresponding {@link JmsIO.Read}. + */ public Read withConnectionFactory(ConnectionFactory connectionFactory) { - return new
[3/3] incubator-beam git commit: [BEAM-716] This closes #1577
[BEAM-716] This closes #1577 Project: http://git-wip-us.apache.org/repos/asf/incubator-beam/repo Commit: http://git-wip-us.apache.org/repos/asf/incubator-beam/commit/1e148cd7 Tree: http://git-wip-us.apache.org/repos/asf/incubator-beam/tree/1e148cd7 Diff: http://git-wip-us.apache.org/repos/asf/incubator-beam/diff/1e148cd7 Branch: refs/heads/master Commit: 1e148cd7d5f12e6742ac57440bf0731460d11b80 Parents: 1c9bf8d 30e14cf Author: Jean-Baptiste OnofréAuthored: Mon Dec 19 07:40:39 2016 +0100 Committer: Jean-Baptiste Onofré Committed: Mon Dec 19 07:40:39 2016 +0100 -- sdks/java/io/jms/pom.xml| 7 + .../java/org/apache/beam/sdk/io/jms/JmsIO.java | 338 +-- 2 files changed, 244 insertions(+), 101 deletions(-) --
[GitHub] incubator-beam-site pull request #110: Update website pull request instructi...
GitHub user jbonofre opened a pull request: https://github.com/apache/incubator-beam-site/pull/110 Update website pull request instructions in the Release Guide You can merge this pull request into a Git repository by running: $ git pull https://github.com/jbonofre/incubator-beam-site RELEASE_GUIDE_WEBSITE_PR_UPDATE Alternatively you can review and apply these changes as the patch at: https://github.com/apache/incubator-beam-site/pull/110.patch To close this pull request, make a commit to your master/trunk branch with (at least) the following in the commit message: This closes #110 commit 8ddeada95adafa0949da8313ff6ab1f780a3a493 Author: Jean-Baptiste Onofré <jbono...@apache.org> Date: 2016-12-15T11:58:30Z Update website pull request instructions in the Release Guide --- If your project is set up for it, you can reply to this email and have your reply appear on GitHub as well. If your project does not have this feature enabled and wishes so, or if the feature is enabled but not working, please contact infrastructure at infrastruct...@apache.org or file a JIRA ticket with INFRA. ---
[GitHub] incubator-beam-site pull request #108: Exclude apex runner translation packa...
GitHub user jbonofre opened a pull request: https://github.com/apache/incubator-beam-site/pull/108 Exclude apex runner translation package from the generated javadoc You can merge this pull request into a Git repository by running: $ git pull https://github.com/jbonofre/incubator-beam-site RELEASE_GUIDE_UPDATE Alternatively you can review and apply these changes as the patch at: https://github.com/apache/incubator-beam-site/pull/108.patch To close this pull request, make a commit to your master/trunk branch with (at least) the following in the commit message: This closes #108 commit c7ac43e62f01950614dc7cc1ec5137cb2bd7a8e6 Author: Jean-Baptiste Onofré <jbono...@apache.org> Date: 2016-12-15T09:35:32Z Update Release Guide: exclude apex runner translation package from the generated javadoc --- If your project is set up for it, you can reply to this email and have your reply appear on GitHub as well. If your project does not have this feature enabled and wishes so, or if the feature is enabled but not working, please contact infrastructure at infrastruct...@apache.org or file a JIRA ticket with INFRA. ---
[GitHub] incubator-beam pull request #1593: Fix JDom malformed comment in Apex runner...
GitHub user jbonofre opened a pull request: https://github.com/apache/incubator-beam/pull/1593 Fix JDom malformed comment in Apex runner. Be sure to do all of the following to help us incorporate your contribution quickly and easily: - [X] Make sure the PR title is formatted like: `[BEAM-] Description of pull request` - [X] Make sure tests pass via `mvn clean verify`. (Even better, enable Travis-CI on your fork and ensure the whole test matrix passes). - [X] Replace `` in the title with the actual Jira issue number, if there is one. - [X] If this contribution is large, please file an Apache [Individual Contributor License Agreement](https://www.apache.org/licenses/icla.txt). --- You can merge this pull request into a Git repository by running: $ git pull https://github.com/jbonofre/incubator-beam APEX_POM Alternatively you can review and apply these changes as the patch at: https://github.com/apache/incubator-beam/pull/1593.patch To close this pull request, make a commit to your master/trunk branch with (at least) the following in the commit message: This closes #1593 commit 40ad5aa92a52f0095f7ed4f68364ff25e59d46f9 Author: Jean-Baptiste Onofré <jbono...@apache.org> Date: 2016-12-13T14:09:03Z Fix JDom malformed comment in Apex runner. --- If your project is set up for it, you can reply to this email and have your reply appear on GitHub as well. If your project does not have this feature enabled and wishes so, or if the feature is enabled but not working, please contact infrastructure at infrastruct...@apache.org or file a JIRA ticket with INFRA. ---
[GitHub] incubator-beam pull request #1577: [BEAM-716] Use AutoValue in JmsIO
GitHub user jbonofre opened a pull request: https://github.com/apache/incubator-beam/pull/1577 [BEAM-716] Use AutoValue in JmsIO Be sure to do all of the following to help us incorporate your contribution quickly and easily: - [X] Make sure the PR title is formatted like: `[BEAM-] Description of pull request` - [X] Make sure tests pass via `mvn clean verify`. (Even better, enable Travis-CI on your fork and ensure the whole test matrix passes). - [X] Replace `` in the title with the actual Jira issue number, if there is one. - [X] If this contribution is large, please file an Apache [Individual Contributor License Agreement](https://www.apache.org/licenses/icla.txt). --- You can merge this pull request into a Git repository by running: $ git pull https://github.com/jbonofre/incubator-beam BEAM-716 Alternatively you can review and apply these changes as the patch at: https://github.com/apache/incubator-beam/pull/1577.patch To close this pull request, make a commit to your master/trunk branch with (at least) the following in the commit message: This closes #1577 commit 4989d83ef92baa493312cbb8b8661496dc1ba583 Author: Jean-Baptiste Onofré <jbono...@apache.org> Date: 2016-12-11T06:43:41Z [BEAM-716] Use AutoValue in JmsIO --- If your project is set up for it, you can reply to this email and have your reply appear on GitHub as well. If your project does not have this feature enabled and wishes so, or if the feature is enabled but not working, please contact infrastructure at infrastruct...@apache.org or file a JIRA ticket with INFRA. ---
[GitHub] incubator-beam pull request #1573: [BEAM-1127] Create an unique source when ...
GitHub user jbonofre opened a pull request: https://github.com/apache/incubator-beam/pull/1573 [BEAM-1127] Create an unique source when using a JMS topic to avoid p⦠Be sure to do all of the following to help us incorporate your contribution quickly and easily: - [X] Make sure the PR title is formatted like: `[BEAM-] Description of pull request` - [X] Make sure tests pass via `mvn clean verify`. (Even better, enable Travis-CI on your fork and ensure the whole test matrix passes). - [X] Replace `` in the title with the actual Jira issue number, if there is one. - [X] If this contribution is large, please file an Apache [Individual Contributor License Agreement](https://www.apache.org/licenses/icla.txt). --- â¦otential messages duplication You can merge this pull request into a Git repository by running: $ git pull https://github.com/jbonofre/incubator-beam BEAM-1127 Alternatively you can review and apply these changes as the patch at: https://github.com/apache/incubator-beam/pull/1573.patch To close this pull request, make a commit to your master/trunk branch with (at least) the following in the commit message: This closes #1573 commit e3a1d73c53a51f1c9ce59c6f3ce2ba5fe6f54f65 Author: Jean-Baptiste Onofré <jbono...@apache.org> Date: 2016-12-10T16:19:44Z [BEAM-1127] Create an unique source when using a JMS topic to avoid potential messages duplication --- If your project is set up for it, you can reply to this email and have your reply appear on GitHub as well. If your project does not have this feature enabled and wishes so, or if the feature is enabled but not working, please contact infrastructure at infrastruct...@apache.org or file a JIRA ticket with INFRA. ---
[2/2] incubator-beam git commit: [BEAM-329] This closes #1532
[BEAM-329] This closes #1532 Project: http://git-wip-us.apache.org/repos/asf/incubator-beam/repo Commit: http://git-wip-us.apache.org/repos/asf/incubator-beam/commit/02bb8c37 Tree: http://git-wip-us.apache.org/repos/asf/incubator-beam/tree/02bb8c37 Diff: http://git-wip-us.apache.org/repos/asf/incubator-beam/diff/02bb8c37 Branch: refs/heads/master Commit: 02bb8c375c48847b1686d70184fb194500a62e8c Parents: 9ccf6db dce3a19 Author: Jean-Baptiste OnofréAuthored: Wed Dec 7 12:51:09 2016 +0100 Committer: Jean-Baptiste Onofré Committed: Wed Dec 7 12:51:09 2016 +0100 -- runners/spark/README.md | 59 +++- .../beam/runners/spark/examples/WordCount.java | 5 +- 2 files changed, 21 insertions(+), 43 deletions(-) --
[1/2] incubator-beam git commit: [BEAM-329] Update Spark runner README.
Repository: incubator-beam Updated Branches: refs/heads/master 9ccf6dbea -> 02bb8c375 [BEAM-329] Update Spark runner README. Project: http://git-wip-us.apache.org/repos/asf/incubator-beam/repo Commit: http://git-wip-us.apache.org/repos/asf/incubator-beam/commit/dce3a196 Tree: http://git-wip-us.apache.org/repos/asf/incubator-beam/tree/dce3a196 Diff: http://git-wip-us.apache.org/repos/asf/incubator-beam/diff/dce3a196 Branch: refs/heads/master Commit: dce3a196a3a26fdd42225520faf3d9084ee48123 Parents: 9ccf6db Author: SelaAuthored: Wed Dec 7 11:20:07 2016 +0200 Committer: Jean-Baptiste Onofré Committed: Wed Dec 7 12:49:21 2016 +0100 -- runners/spark/README.md | 59 +++- .../beam/runners/spark/examples/WordCount.java | 5 +- 2 files changed, 21 insertions(+), 43 deletions(-) -- http://git-wip-us.apache.org/repos/asf/incubator-beam/blob/dce3a196/runners/spark/README.md -- diff --git a/runners/spark/README.md b/runners/spark/README.md index ef42fa7..aad65b3 100644 --- a/runners/spark/README.md +++ b/runners/spark/README.md @@ -38,32 +38,25 @@ with Apache Spark. This runner allows to execute both batch and streaming pipeli - Side inputs/outputs - Encoding -### Sources and Sinks - -- Text -- Hadoop -- Avro -- Kafka - ### Fault-Tolerance The Spark runner fault-tolerance guarantees the same guarantees as [Apache Spark](http://spark.apache.org/). ### Monitoring -The Spark runner supports monitoring via Beam Aggregators implemented on top of Spark's [Accumulators](http://spark.apache.org/docs/latest/programming-guide.html#accumulators). -Spark also provides a web UI for monitoring, more details [here](http://spark.apache.org/docs/latest/monitoring.html). +The Spark runner supports user-defined counters via Beam Aggregators implemented on top of Spark's [Accumulators](http://spark.apache.org/docs/1.6.3/programming-guide.html#accumulators). +The Aggregators (defined by the pipeline author) and Spark's internal metrics are reported using Spark's [metrics system](http://spark.apache.org/docs/1.6.3/monitoring.html#metrics). +Spark also provides a web UI for monitoring, more details [here](http://spark.apache.org/docs/1.6.3/monitoring.html). ## Beam Model support ### Batch -The Spark runner provides support for batch processing of Beam bounded PCollections as Spark [RDD](http://spark.apache.org/docs/latest/programming-guide.html#resilient-distributed-datasets-rdds)s. +The Spark runner provides full support for the Beam Model in batch processing via Spark [RDD](http://spark.apache.org/docs/1.6.3/programming-guide.html#resilient-distributed-datasets-rdds)s. ### Streaming -The Spark runner currently provides partial support for stream processing of Beam unbounded PCollections as Spark [DStream](http://spark.apache.org/docs/latest/streaming-programming-guide.html#discretized-streams-dstreams)s. -Currently, both *FixedWindows* and *SlidingWindows* are supported, but only with processing-time triggers and discarding pane. +Providing full support for the Beam Model in streaming pipelines is under development. To follow-up you can subscribe to our [mailing list](http://beam.incubator.apache.org/get-started/support/). ### issue tracking @@ -84,19 +77,21 @@ Then switch to the newly created directory and run Maven to build the Apache Bea Now Apache Beam and the Spark Runner are installed in your local maven repository. -If we wanted to run a Beam pipeline with the default options of a single threaded Spark -instance in local mode, we would do the following: +If we wanted to run a Beam pipeline with the default options of a Spark instance in local mode, +we would do the following: Pipeline p = -EvaluationResult result = SparkRunner.create().run(p); +PipelineResult result = p.run(); +result.waitUntilFinish(); To create a pipeline runner to run against a different Spark cluster, with a custom master url we would do the following: -Pipeline p = -SparkPipelineOptions options = SparkPipelineOptionsFactory.create(); +SparkPipelineOptions options = PipelineOptionsFactory.as(SparkPipelineOptions.class); options.setSparkMaster("spark://host:port"); -EvaluationResult result = SparkRunner.create(options).run(p); +Pipeline p = +PipelineResult result = p.run(); +result.waitUntilFinish(); ## Word Count Example @@ -108,12 +103,11 @@ Switch to the Spark runner directory: cd runners/spark -Then run the [word count example][wc] from the SDK using a single threaded Spark instance -in local mode: +Then run the [word count example][wc] from the SDK using a Spark instance in local mode: -mvn exec:exec
[GitHub] incubator-beam pull request #1531: [BEAM-1094] Set test scope for Kafka IO a...
GitHub user jbonofre opened a pull request: https://github.com/apache/incubator-beam/pull/1531 [BEAM-1094] Set test scope for Kafka IO and junit Be sure to do all of the following to help us incorporate your contribution quickly and easily: - [X] Make sure the PR title is formatted like: `[BEAM-] Description of pull request` - [X] Make sure tests pass via `mvn clean verify`. (Even better, enable Travis-CI on your fork and ensure the whole test matrix passes). - [X] Replace `` in the title with the actual Jira issue number, if there is one. - [X] If this contribution is large, please file an Apache [Individual Contributor License Agreement](https://www.apache.org/licenses/icla.txt). --- You can merge this pull request into a Git repository by running: $ git pull https://github.com/jbonofre/incubator-beam BEAM-1094 Alternatively you can review and apply these changes as the patch at: https://github.com/apache/incubator-beam/pull/1531.patch To close this pull request, make a commit to your master/trunk branch with (at least) the following in the commit message: This closes #1531 commit 346c0b528297ab39bfa021ee052dcee48f56953d Author: Jean-Baptiste Onofré <jbono...@apache.org> Date: 2016-12-07T07:37:33Z [BEAM-1094] Set test scope for Kafka IO and junit --- If your project is set up for it, you can reply to this email and have your reply appear on GitHub as well. If your project does not have this feature enabled and wishes so, or if the feature is enabled but not working, please contact infrastructure at infrastruct...@apache.org or file a JIRA ticket with INFRA. ---
[1/2] incubator-beam git commit: [BEAM-961] Add starting number to CountingInput
Repository: incubator-beam Updated Branches: refs/heads/master 1efda59ab -> 493c04faa [BEAM-961] Add starting number to CountingInput Project: http://git-wip-us.apache.org/repos/asf/incubator-beam/repo Commit: http://git-wip-us.apache.org/repos/asf/incubator-beam/commit/41ae08bf Tree: http://git-wip-us.apache.org/repos/asf/incubator-beam/tree/41ae08bf Diff: http://git-wip-us.apache.org/repos/asf/incubator-beam/diff/41ae08bf Branch: refs/heads/master Commit: 41ae08bf18525f52b03252dee783505ae400911e Parents: 1efda59 Author: Vladisav JelisavcicAuthored: Sun Dec 4 10:42:28 2016 +0100 Committer: Jean-Baptiste Onofré Committed: Tue Dec 6 11:01:45 2016 +0100 -- .../org/apache/beam/sdk/io/CountingInput.java | 42 .../org/apache/beam/sdk/io/CountingSource.java | 11 + .../apache/beam/sdk/io/CountingInputTest.java | 42 +++- 3 files changed, 76 insertions(+), 19 deletions(-) -- http://git-wip-us.apache.org/repos/asf/incubator-beam/blob/41ae08bf/sdks/java/core/src/main/java/org/apache/beam/sdk/io/CountingInput.java -- diff --git a/sdks/java/core/src/main/java/org/apache/beam/sdk/io/CountingInput.java b/sdks/java/core/src/main/java/org/apache/beam/sdk/io/CountingInput.java index f479215..456d291 100644 --- a/sdks/java/core/src/main/java/org/apache/beam/sdk/io/CountingInput.java +++ b/sdks/java/core/src/main/java/org/apache/beam/sdk/io/CountingInput.java @@ -35,7 +35,7 @@ import org.joda.time.Instant; /** * A {@link PTransform} that produces longs. When used to produce a * {@link IsBounded#BOUNDED bounded} {@link PCollection}, {@link CountingInput} starts at {@code 0} - * and counts up to a specified maximum. When used to produce an + * or starting value, and counts up to a specified maximum. When used to produce an * {@link IsBounded#UNBOUNDED unbounded} {@link PCollection}, it counts up to {@link Long#MAX_VALUE} * and then never produces more output. (In practice, this limit should never be reached.) * @@ -43,7 +43,8 @@ import org.joda.time.Instant; * {@link OffsetBasedSource.OffsetBasedReader}, so it performs efficient initial splitting and it * supports dynamic work rebalancing. * - * To produce a bounded {@code PCollection}, use {@link CountingInput#upTo(long)}: + * To produce a bounded {@code PCollection} starting from {@code 0}, + * use {@link CountingInput#upTo(long)}: * * {@code * Pipeline p = ... @@ -51,6 +52,9 @@ import org.joda.time.Instant; * PCollection bounded = p.apply(producer); * } * + * To produce a bounded {@code PCollection} starting from {@code startOffset}, + * use {@link CountingInput#forSubrange(long, long)} instead. + * * To produce an unbounded {@code PCollection}, use {@link CountingInput#unbounded()}, * calling {@link UnboundedCountingInput#withTimestampFn(SerializableFunction)} to provide values * with timestamps other than {@link Instant#now}. @@ -76,6 +80,16 @@ public class CountingInput { } /** + * Creates a {@link BoundedCountingInput} that will produce elements + * starting from {@code startIndex} to {@code endIndex - 1}. + */ + public static BoundedCountingInput forSubrange(long startIndex, long endIndex) { +checkArgument(endIndex > startIndex, "endIndex (%s) must be greater than startIndex (%s)", +endIndex, startIndex); +return new BoundedCountingInput(startIndex, endIndex); + } + + /** * Creates an {@link UnboundedCountingInput} that will produce numbers starting from {@code 0} up * to {@link Long#MAX_VALUE}. * @@ -102,23 +116,35 @@ public class CountingInput { * 0. */ public static class BoundedCountingInput extends PTransform { -private final long numElements; +private final long startIndex; +private final long endIndex; private BoundedCountingInput(long numElements) { - this.numElements = numElements; + this.endIndex = numElements; + this.startIndex = 0; +} + +private BoundedCountingInput(long startIndex, long endIndex) { + this.endIndex = endIndex; + this.startIndex = startIndex; } -@SuppressWarnings("deprecation") @Override public PCollection apply(PBegin begin) { - return begin.apply(Read.from(CountingSource.upTo(numElements))); + return begin.apply(Read.from(CountingSource.createSourceForSubrange(startIndex, endIndex))); } @Override public void populateDisplayData(DisplayData.Builder builder) { super.populateDisplayData(builder); - builder.add(DisplayData.item("upTo", numElements) -.withLabel("Count Up To")); + + if (startIndex == 0) { +builder.add(DisplayData.item("upTo", endIndex) +
[2/2] incubator-beam git commit: [BEAM-961] This closes #1505
[BEAM-961] This closes #1505 Project: http://git-wip-us.apache.org/repos/asf/incubator-beam/repo Commit: http://git-wip-us.apache.org/repos/asf/incubator-beam/commit/493c04fa Tree: http://git-wip-us.apache.org/repos/asf/incubator-beam/tree/493c04fa Diff: http://git-wip-us.apache.org/repos/asf/incubator-beam/diff/493c04fa Branch: refs/heads/master Commit: 493c04faa12b8bb7483e5805bd4eb4e3280feaca Parents: 1efda59 41ae08b Author: Jean-Baptiste OnofréAuthored: Tue Dec 6 11:17:02 2016 +0100 Committer: Jean-Baptiste Onofré Committed: Tue Dec 6 11:17:02 2016 +0100 -- .../org/apache/beam/sdk/io/CountingInput.java | 42 .../org/apache/beam/sdk/io/CountingSource.java | 11 + .../apache/beam/sdk/io/CountingInputTest.java | 42 +++- 3 files changed, 76 insertions(+), 19 deletions(-) --
[GitHub] incubator-beam pull request #1515: [BEAM-293] StreamingOptions doesn't exten...
GitHub user jbonofre opened a pull request: https://github.com/apache/incubator-beam/pull/1515 [BEAM-293] StreamingOptions doesn't extend GcpOptions Be sure to do all of the following to help us incorporate your contribution quickly and easily: - [X] Make sure the PR title is formatted like: `[BEAM-] Description of pull request` - [X] Make sure tests pass via `mvn clean verify`. (Even better, enable Travis-CI on your fork and ensure the whole test matrix passes). - [X] Replace `` in the title with the actual Jira issue number, if there is one. - [X] If this contribution is large, please file an Apache [Individual Contributor License Agreement](https://www.apache.org/licenses/icla.txt). --- You can merge this pull request into a Git repository by running: $ git pull https://github.com/jbonofre/incubator-beam BEAM-293 Alternatively you can review and apply these changes as the patch at: https://github.com/apache/incubator-beam/pull/1515.patch To close this pull request, make a commit to your master/trunk branch with (at least) the following in the commit message: This closes #1515 commit 5c57e4fda34a1f259ee27263e3b12f94761a178b Author: Jean-Baptiste Onofré <jbono...@apache.org> Date: 2016-12-06T05:20:07Z [BEAM-293] StreamingOptions doesn't extend GcpOptions --- If your project is set up for it, you can reply to this email and have your reply appear on GitHub as well. If your project does not have this feature enabled and wishes so, or if the feature is enabled but not working, please contact infrastructure at infrastruct...@apache.org or file a JIRA ticket with INFRA. ---
[2/2] incubator-beam git commit: [BEAM-1057] This closes #1444
[BEAM-1057] This closes #1444 Project: http://git-wip-us.apache.org/repos/asf/incubator-beam/repo Commit: http://git-wip-us.apache.org/repos/asf/incubator-beam/commit/ef9871c3 Tree: http://git-wip-us.apache.org/repos/asf/incubator-beam/tree/ef9871c3 Diff: http://git-wip-us.apache.org/repos/asf/incubator-beam/diff/ef9871c3 Branch: refs/heads/master Commit: ef9871c36f24cde537c4067357f534afa4a920a9 Parents: c22b97d 58916b9 Author: Jean-Baptiste OnofréAuthored: Sun Dec 4 07:21:28 2016 +0100 Committer: Jean-Baptiste Onofré Committed: Sun Dec 4 07:21:28 2016 +0100 -- .../org/apache/beam/sdk/io/jdbc/JdbcIOTest.java | 26 +++- 1 file changed, 25 insertions(+), 1 deletion(-) --
[1/2] incubator-beam git commit: [BEAM-1057] Fix JDBC test derby startup issues
Repository: incubator-beam Updated Branches: refs/heads/master c22b97dd1 -> ef9871c36 [BEAM-1057] Fix JDBC test derby startup issues Project: http://git-wip-us.apache.org/repos/asf/incubator-beam/repo Commit: http://git-wip-us.apache.org/repos/asf/incubator-beam/commit/58916b94 Tree: http://git-wip-us.apache.org/repos/asf/incubator-beam/tree/58916b94 Diff: http://git-wip-us.apache.org/repos/asf/incubator-beam/diff/58916b94 Branch: refs/heads/master Commit: 58916b94ada142d63d16a315da58f88184995f10 Parents: c22b97d Author: Daniel KulpAuthored: Mon Nov 28 16:47:37 2016 -0500 Committer: Jean-Baptiste Onofré Committed: Sun Dec 4 07:17:42 2016 +0100 -- .../org/apache/beam/sdk/io/jdbc/JdbcIOTest.java | 26 +++- 1 file changed, 25 insertions(+), 1 deletion(-) -- http://git-wip-us.apache.org/repos/asf/incubator-beam/blob/58916b94/sdks/java/io/jdbc/src/test/java/org/apache/beam/sdk/io/jdbc/JdbcIOTest.java -- diff --git a/sdks/java/io/jdbc/src/test/java/org/apache/beam/sdk/io/jdbc/JdbcIOTest.java b/sdks/java/io/jdbc/src/test/java/org/apache/beam/sdk/io/jdbc/JdbcIOTest.java index fe574af..d09929d 100644 --- a/sdks/java/io/jdbc/src/test/java/org/apache/beam/sdk/io/jdbc/JdbcIOTest.java +++ b/sdks/java/io/jdbc/src/test/java/org/apache/beam/sdk/io/jdbc/JdbcIOTest.java @@ -20,7 +20,9 @@ package org.apache.beam.sdk.io.jdbc; import static org.junit.Assert.assertEquals; import static org.junit.Assert.assertTrue; +import java.io.PrintWriter; import java.io.Serializable; +import java.io.StringWriter; import java.net.InetAddress; import java.net.ServerSocket; import java.sql.Connection; @@ -73,7 +75,29 @@ public class JdbcIOTest implements Serializable { System.setProperty("derby.stream.error.file", "target/derby.log"); derbyServer = new NetworkServerControl(InetAddress.getByName("localhost"), port); -derbyServer.start(null); +StringWriter out = new StringWriter(); +derbyServer.start(new PrintWriter(out)); +boolean started = false; +int count = 0; +// Use two different methods to detect when server is started: +// 1) Check the server stdout for the "started" string +// 2) wait up to 15 seconds for the derby server to start based on a ping +// on faster machines and networks, this may return very quick, but on slower +// networks where the DNS lookups are slow, this may take a little time +while (!started && count < 30) { + if (out.toString().contains("started")) { +started = true; + } else { +count++; +Thread.sleep(500); +try { + derbyServer.ping(); + started = true; +} catch (Throwable t) { + //ignore, still trying to start +} + } +} dataSource = new ClientDataSource(); dataSource.setCreateDatabase("create");
[2/2] incubator-beam git commit: [BEAM-918] This closes #1370
[BEAM-918] This closes #1370 Project: http://git-wip-us.apache.org/repos/asf/incubator-beam/repo Commit: http://git-wip-us.apache.org/repos/asf/incubator-beam/commit/0c875ba7 Tree: http://git-wip-us.apache.org/repos/asf/incubator-beam/tree/0c875ba7 Diff: http://git-wip-us.apache.org/repos/asf/incubator-beam/diff/0c875ba7 Branch: refs/heads/master Commit: 0c875ba704c2501c3215ffd588d02d2e4117ded2 Parents: 711c680 d99829d Author: Jean-Baptiste OnofréAuthored: Thu Dec 1 11:43:36 2016 +0100 Committer: Jean-Baptiste Onofré Committed: Thu Dec 1 11:43:36 2016 +0100 -- .../runners/spark/SparkPipelineOptions.java | 5 ++ .../spark/translation/BoundedDataset.java | 5 +- .../beam/runners/spark/translation/Dataset.java | 2 +- .../spark/translation/EvaluationContext.java| 10 +++- .../translation/StorageLevelPTransform.java | 43 +++ .../spark/translation/TransformTranslator.java | 27 ++ .../translation/streaming/UnboundedDataset.java | 13 - .../spark/translation/StorageLevelTest.java | 56 8 files changed, 155 insertions(+), 6 deletions(-) --
[1/2] incubator-beam git commit: [BEAM-918] Allow users to define the storage level via pipeline options
Repository: incubator-beam Updated Branches: refs/heads/master 711c68092 -> 0c875ba70 [BEAM-918] Allow users to define the storage level via pipeline options Project: http://git-wip-us.apache.org/repos/asf/incubator-beam/repo Commit: http://git-wip-us.apache.org/repos/asf/incubator-beam/commit/d99829dd Tree: http://git-wip-us.apache.org/repos/asf/incubator-beam/tree/d99829dd Diff: http://git-wip-us.apache.org/repos/asf/incubator-beam/diff/d99829dd Branch: refs/heads/master Commit: d99829dd99db4090ceb7e5eefce50ee513c5458e Parents: 711c680 Author: Jean-Baptiste OnofréAuthored: Thu Nov 17 12:38:00 2016 +0100 Committer: Jean-Baptiste Onofré Committed: Thu Dec 1 11:38:25 2016 +0100 -- .../runners/spark/SparkPipelineOptions.java | 5 ++ .../spark/translation/BoundedDataset.java | 5 +- .../beam/runners/spark/translation/Dataset.java | 2 +- .../spark/translation/EvaluationContext.java| 10 +++- .../translation/StorageLevelPTransform.java | 43 +++ .../spark/translation/TransformTranslator.java | 27 ++ .../translation/streaming/UnboundedDataset.java | 13 - .../spark/translation/StorageLevelTest.java | 56 8 files changed, 155 insertions(+), 6 deletions(-) -- http://git-wip-us.apache.org/repos/asf/incubator-beam/blob/d99829dd/runners/spark/src/main/java/org/apache/beam/runners/spark/SparkPipelineOptions.java -- diff --git a/runners/spark/src/main/java/org/apache/beam/runners/spark/SparkPipelineOptions.java b/runners/spark/src/main/java/org/apache/beam/runners/spark/SparkPipelineOptions.java index 0fd790e..3f8b379 100644 --- a/runners/spark/src/main/java/org/apache/beam/runners/spark/SparkPipelineOptions.java +++ b/runners/spark/src/main/java/org/apache/beam/runners/spark/SparkPipelineOptions.java @@ -44,6 +44,11 @@ public interface SparkPipelineOptions Long getBatchIntervalMillis(); void setBatchIntervalMillis(Long batchInterval); + @Description("Batch default storage level") + @Default.String("MEMORY_ONLY") + String getStorageLevel(); + void setStorageLevel(String storageLevel); + @Description("Minimum time to spend on read, for each micro-batch.") @Default.Long(200) Long getMinReadTimeMillis(); http://git-wip-us.apache.org/repos/asf/incubator-beam/blob/d99829dd/runners/spark/src/main/java/org/apache/beam/runners/spark/translation/BoundedDataset.java -- diff --git a/runners/spark/src/main/java/org/apache/beam/runners/spark/translation/BoundedDataset.java b/runners/spark/src/main/java/org/apache/beam/runners/spark/translation/BoundedDataset.java index 774efb9..1cfb0e0 100644 --- a/runners/spark/src/main/java/org/apache/beam/runners/spark/translation/BoundedDataset.java +++ b/runners/spark/src/main/java/org/apache/beam/runners/spark/translation/BoundedDataset.java @@ -32,6 +32,7 @@ import org.apache.beam.sdk.values.PCollection; import org.apache.spark.api.java.JavaRDD; import org.apache.spark.api.java.JavaRDDLike; import org.apache.spark.api.java.JavaSparkContext; +import org.apache.spark.storage.StorageLevel; /** * Holds an RDD or values for deferred conversion to an RDD if needed. PCollections are sometimes @@ -97,8 +98,8 @@ public class BoundedDataset implements Dataset { } @Override - public void cache() { -rdd.cache(); + public void cache(String storageLevel) { +rdd.persist(StorageLevel.fromString(storageLevel)); } @Override http://git-wip-us.apache.org/repos/asf/incubator-beam/blob/d99829dd/runners/spark/src/main/java/org/apache/beam/runners/spark/translation/Dataset.java -- diff --git a/runners/spark/src/main/java/org/apache/beam/runners/spark/translation/Dataset.java b/runners/spark/src/main/java/org/apache/beam/runners/spark/translation/Dataset.java index 36b03fe..b5d550e 100644 --- a/runners/spark/src/main/java/org/apache/beam/runners/spark/translation/Dataset.java +++ b/runners/spark/src/main/java/org/apache/beam/runners/spark/translation/Dataset.java @@ -26,7 +26,7 @@ import java.io.Serializable; */ public interface Dataset extends Serializable { - void cache(); + void cache(String storageLevel); void action(); http://git-wip-us.apache.org/repos/asf/incubator-beam/blob/d99829dd/runners/spark/src/main/java/org/apache/beam/runners/spark/translation/EvaluationContext.java -- diff --git a/runners/spark/src/main/java/org/apache/beam/runners/spark/translation/EvaluationContext.java b/runners/spark/src/main/java/org/apache/beam/runners/spark/translation/EvaluationContext.java index
[GitHub] incubator-beam pull request #1451: [BEAM-1059] Create ExecIO
GitHub user jbonofre opened a pull request: https://github.com/apache/incubator-beam/pull/1451 [BEAM-1059] Create ExecIO Be sure to do all of the following to help us incorporate your contribution quickly and easily: - [X] Make sure the PR title is formatted like: `[BEAM-] Description of pull request` - [X] Make sure tests pass via `mvn clean verify`. (Even better, enable Travis-CI on your fork and ensure the whole test matrix passes). - [X] Replace `` in the title with the actual Jira issue number, if there is one. - [X] If this contribution is large, please file an Apache [Individual Contributor License Agreement](https://www.apache.org/licenses/icla.txt). --- You can merge this pull request into a Git repository by running: $ git pull https://github.com/jbonofre/incubator-beam BEAM-1059-EXECIO Alternatively you can review and apply these changes as the patch at: https://github.com/apache/incubator-beam/pull/1451.patch To close this pull request, make a commit to your master/trunk branch with (at least) the following in the commit message: This closes #1451 commit 2e2ae69b9a2cd2ea2284823d5c0c6bf2d131eda3 Author: Jean-Baptiste Onofré <jbono...@apache.org> Date: 2016-11-28T11:36:51Z [BEAM-1059] Create ExecIO --- If your project is set up for it, you can reply to this email and have your reply appear on GitHub as well. If your project does not have this feature enabled and wishes so, or if the feature is enabled but not working, please contact infrastructure at infrastruct...@apache.org or file a JIRA ticket with INFRA. ---
[2/2] incubator-beam git commit: [BEAM-1006] This closes #1387
[BEAM-1006] This closes #1387 Project: http://git-wip-us.apache.org/repos/asf/incubator-beam/repo Commit: http://git-wip-us.apache.org/repos/asf/incubator-beam/commit/3f16f266 Tree: http://git-wip-us.apache.org/repos/asf/incubator-beam/tree/3f16f266 Diff: http://git-wip-us.apache.org/repos/asf/incubator-beam/diff/3f16f266 Branch: refs/heads/master Commit: 3f16f26600ac6e552f379c94cfae1bc57ed5f5f0 Parents: aeff1d5 b453457 Author: Jean-Baptiste OnofréAuthored: Tue Nov 29 08:15:08 2016 +0100 Committer: Jean-Baptiste Onofré Committed: Tue Nov 29 08:15:08 2016 +0100 -- .../main/java/org/apache/beam/sdk/io/mongodb/MongoDbIO.java| 6 +++--- 1 file changed, 3 insertions(+), 3 deletions(-) --
[1/2] incubator-beam git commit: [BEAM-1006] Fixing the MongoDbIO splitKeysToFilters, calling to string on the BSON document prefixed Document to the string, updated it to just get the value of ID
Repository: incubator-beam Updated Branches: refs/heads/master aeff1d5c2 -> 3f16f2660 [BEAM-1006] Fixing the MongoDbIO splitKeysToFilters, calling to string on the BSON document prefixed Document to the string, updated it to just get the value of ID Project: http://git-wip-us.apache.org/repos/asf/incubator-beam/repo Commit: http://git-wip-us.apache.org/repos/asf/incubator-beam/commit/b453457f Tree: http://git-wip-us.apache.org/repos/asf/incubator-beam/tree/b453457f Diff: http://git-wip-us.apache.org/repos/asf/incubator-beam/diff/b453457f Branch: refs/heads/master Commit: b453457f5699963f9a72129ab86ef18b71c04e61 Parents: aeff1d5 Author: DavidBAuthored: Fri Nov 18 15:11:47 2016 + Committer: Jean-Baptiste Onofré Committed: Tue Nov 29 07:50:40 2016 +0100 -- .../main/java/org/apache/beam/sdk/io/mongodb/MongoDbIO.java| 6 +++--- 1 file changed, 3 insertions(+), 3 deletions(-) -- http://git-wip-us.apache.org/repos/asf/incubator-beam/blob/b453457f/sdks/java/io/mongodb/src/main/java/org/apache/beam/sdk/io/mongodb/MongoDbIO.java -- diff --git a/sdks/java/io/mongodb/src/main/java/org/apache/beam/sdk/io/mongodb/MongoDbIO.java b/sdks/java/io/mongodb/src/main/java/org/apache/beam/sdk/io/mongodb/MongoDbIO.java index cd5cdaf..70239e6 100644 --- a/sdks/java/io/mongodb/src/main/java/org/apache/beam/sdk/io/mongodb/MongoDbIO.java +++ b/sdks/java/io/mongodb/src/main/java/org/apache/beam/sdk/io/mongodb/MongoDbIO.java @@ -302,12 +302,12 @@ public class MongoDbIO { ArrayList filters = new ArrayList<>(); String lowestBound = null; // lower boundary (previous split in the iteration) for (int i = 0; i < splitKeys.size(); i++) { -String splitKey = splitKeys.get(i).toString(); -String rangeFilter = null; +String splitKey = splitKeys.get(i).get("_id").toString(); +String rangeFilter; if (i == 0) { // this is the first split in the list, the filter defines // the range from the beginning up to this split - rangeFilter = String.format("{ $and: [ {\"_id\":{$lte:Objectd(\"%s\")}}", + rangeFilter = String.format("{ $and: [ {\"_id\":{$lte:ObjectId(\"%s\")}}", splitKey); } else if (i == splitKeys.size() - 1) { // this is the last split in the list, the filter defines
incubator-beam git commit: [BEAM-1049] This closes #1437 [Forced Update!]
Repository: incubator-beam Updated Branches: refs/heads/master 93b6050dc -> 9fbd2d24e (forced update) [BEAM-1049] This closes #1437 Project: http://git-wip-us.apache.org/repos/asf/incubator-beam/repo Commit: http://git-wip-us.apache.org/repos/asf/incubator-beam/commit/9fbd2d24 Tree: http://git-wip-us.apache.org/repos/asf/incubator-beam/tree/9fbd2d24 Diff: http://git-wip-us.apache.org/repos/asf/incubator-beam/diff/9fbd2d24 Branch: refs/heads/master Commit: 9fbd2d24e532221fdbe7ed29f62f315fd4a1eab1 Parents: 8cc43aa f600559 Author: Jean-Baptiste OnofréAuthored: Sun Nov 27 18:53:10 2016 +0100 Committer: Jean-Baptiste Onofré Committed: Mon Nov 28 12:37:27 2016 +0100 -- runners/spark/pom.xml | 2 +- .../beam/runners/spark/aggregators/AccumulatorSingleton.java | 2 +- .../translation/streaming/SparkRunnerStreamingContextFactory.java | 2 +- 3 files changed, 3 insertions(+), 3 deletions(-) --
[1/2] incubator-beam git commit: [BEAM-1049] Update spark version to 1.6.3
Repository: incubator-beam Updated Branches: refs/heads/master 8cc43aa70 -> 93b6050dc [BEAM-1049] Update spark version to 1.6.3 Project: http://git-wip-us.apache.org/repos/asf/incubator-beam/repo Commit: http://git-wip-us.apache.org/repos/asf/incubator-beam/commit/f6005593 Tree: http://git-wip-us.apache.org/repos/asf/incubator-beam/tree/f6005593 Diff: http://git-wip-us.apache.org/repos/asf/incubator-beam/diff/f6005593 Branch: refs/heads/master Commit: f6005593ccf6bdef7c975622d5af39792c7db44c Parents: 8cc43aa Author: Ismaël MejÃaAuthored: Sun Nov 27 11:39:08 2016 +0100 Committer: Ismaël MejÃa Committed: Sun Nov 27 14:49:06 2016 +0100 -- runners/spark/pom.xml | 2 +- .../beam/runners/spark/aggregators/AccumulatorSingleton.java | 2 +- .../translation/streaming/SparkRunnerStreamingContextFactory.java | 2 +- 3 files changed, 3 insertions(+), 3 deletions(-) -- http://git-wip-us.apache.org/repos/asf/incubator-beam/blob/f6005593/runners/spark/pom.xml -- diff --git a/runners/spark/pom.xml b/runners/spark/pom.xml index 88223e2..da7a72a 100644 --- a/runners/spark/pom.xml +++ b/runners/spark/pom.xml @@ -34,7 +34,7 @@ UTF-8 UTF-8 -1.6.2 +1.6.3 2.2.0 0.9.0.1 3.1.2 http://git-wip-us.apache.org/repos/asf/incubator-beam/blob/f6005593/runners/spark/src/main/java/org/apache/beam/runners/spark/aggregators/AccumulatorSingleton.java -- diff --git a/runners/spark/src/main/java/org/apache/beam/runners/spark/aggregators/AccumulatorSingleton.java b/runners/spark/src/main/java/org/apache/beam/runners/spark/aggregators/AccumulatorSingleton.java index 758372e..bc7105f 100644 --- a/runners/spark/src/main/java/org/apache/beam/runners/spark/aggregators/AccumulatorSingleton.java +++ b/runners/spark/src/main/java/org/apache/beam/runners/spark/aggregators/AccumulatorSingleton.java @@ -24,7 +24,7 @@ import org.apache.spark.api.java.JavaSparkContext; /** * For resilience, {@link Accumulator}s are required to be wrapped in a Singleton. - * @see https://spark.apache.org/docs/1.6.2/streaming-programming-guide.html#accumulators-and-broadcast-variables;>accumulators + * @see https://spark.apache.org/docs/1.6.3/streaming-programming-guide.html#accumulators-and-broadcast-variables;>accumulators */ public class AccumulatorSingleton { http://git-wip-us.apache.org/repos/asf/incubator-beam/blob/f6005593/runners/spark/src/main/java/org/apache/beam/runners/spark/translation/streaming/SparkRunnerStreamingContextFactory.java -- diff --git a/runners/spark/src/main/java/org/apache/beam/runners/spark/translation/streaming/SparkRunnerStreamingContextFactory.java b/runners/spark/src/main/java/org/apache/beam/runners/spark/translation/streaming/SparkRunnerStreamingContextFactory.java index af90ff1..d069a11 100644 --- a/runners/spark/src/main/java/org/apache/beam/runners/spark/translation/streaming/SparkRunnerStreamingContextFactory.java +++ b/runners/spark/src/main/java/org/apache/beam/runners/spark/translation/streaming/SparkRunnerStreamingContextFactory.java @@ -40,7 +40,7 @@ import org.slf4j.LoggerFactory; /** * A {@link JavaStreamingContext} factory for resilience. - * @see https://spark.apache.org/docs/1.6.2/streaming-programming-guide.html#how-to-configure-checkpointing;>how-to-configure-checkpointing + * @see https://spark.apache.org/docs/1.6.3/streaming-programming-guide.html#how-to-configure-checkpointing;>how-to-configure-checkpointing */ public class SparkRunnerStreamingContextFactory implements JavaStreamingContextFactory { private static final Logger LOG =
[2/2] incubator-beam git commit: [BEAM-1049] This closes #1437
[BEAM-1049] This closes #1437 Project: http://git-wip-us.apache.org/repos/asf/incubator-beam/repo Commit: http://git-wip-us.apache.org/repos/asf/incubator-beam/commit/93b6050d Tree: http://git-wip-us.apache.org/repos/asf/incubator-beam/tree/93b6050d Diff: http://git-wip-us.apache.org/repos/asf/incubator-beam/diff/93b6050d Branch: refs/heads/master Commit: 93b6050dcc32c6a2821baea7f255f3bd18a07760 Parents: 8cc43aa f600559 Author: Etienne Chauchot and Jean-Baptiste OnofréAuthored: Sun Nov 27 18:53:10 2016 +0100 Committer: Etienne Chauchot and Jean-Baptiste Onofré Committed: Sun Nov 27 18:53:10 2016 +0100 -- runners/spark/pom.xml | 2 +- .../beam/runners/spark/aggregators/AccumulatorSingleton.java | 2 +- .../translation/streaming/SparkRunnerStreamingContextFactory.java | 2 +- 3 files changed, 3 insertions(+), 3 deletions(-) --
[2/2] incubator-beam git commit: [BEAM-959] This closes #1374
[BEAM-959] This closes #1374 Project: http://git-wip-us.apache.org/repos/asf/incubator-beam/repo Commit: http://git-wip-us.apache.org/repos/asf/incubator-beam/commit/3e4b2fd0 Tree: http://git-wip-us.apache.org/repos/asf/incubator-beam/tree/3e4b2fd0 Diff: http://git-wip-us.apache.org/repos/asf/incubator-beam/diff/3e4b2fd0 Branch: refs/heads/master Commit: 3e4b2fd0d96ff2757de7782b7c80dc1881eb451b Parents: 6d0c205 7b314aa Author: Jean-Baptiste OnofréAuthored: Thu Nov 24 10:31:52 2016 +0100 Committer: Jean-Baptiste Onofré Committed: Thu Nov 24 10:31:52 2016 +0100 -- .../org/apache/beam/sdk/io/jdbc/JdbcIO.java | 52 +--- 1 file changed, 35 insertions(+), 17 deletions(-) --
[1/2] incubator-beam git commit: [BEAM-959] Improve validation messages in JdbcIO
Repository: incubator-beam Updated Branches: refs/heads/master 6d0c205a3 -> 3e4b2fd0d [BEAM-959] Improve validation messages in JdbcIO Project: http://git-wip-us.apache.org/repos/asf/incubator-beam/repo Commit: http://git-wip-us.apache.org/repos/asf/incubator-beam/commit/7b314aad Tree: http://git-wip-us.apache.org/repos/asf/incubator-beam/tree/7b314aad Diff: http://git-wip-us.apache.org/repos/asf/incubator-beam/diff/7b314aad Branch: refs/heads/master Commit: 7b314aad1c7c62ad61e09e610c60f53ac056d75d Parents: 6d0c205 Author: Jean-Baptiste OnofréAuthored: Thu Nov 17 17:07:21 2016 +0100 Committer: Jean-Baptiste Onofré Committed: Thu Nov 24 08:50:01 2016 +0100 -- .../org/apache/beam/sdk/io/jdbc/JdbcIO.java | 52 +--- 1 file changed, 35 insertions(+), 17 deletions(-) -- http://git-wip-us.apache.org/repos/asf/incubator-beam/blob/7b314aad/sdks/java/io/jdbc/src/main/java/org/apache/beam/sdk/io/jdbc/JdbcIO.java -- diff --git a/sdks/java/io/jdbc/src/main/java/org/apache/beam/sdk/io/jdbc/JdbcIO.java b/sdks/java/io/jdbc/src/main/java/org/apache/beam/sdk/io/jdbc/JdbcIO.java index 0e0703f..9644a65 100644 --- a/sdks/java/io/jdbc/src/main/java/org/apache/beam/sdk/io/jdbc/JdbcIO.java +++ b/sdks/java/io/jdbc/src/main/java/org/apache/beam/sdk/io/jdbc/JdbcIO.java @@ -18,7 +18,7 @@ package org.apache.beam.sdk.io.jdbc; import static com.google.common.base.Preconditions.checkArgument; -import static com.google.common.base.Preconditions.checkNotNull; +import static com.google.common.base.Preconditions.checkState; import com.google.auto.value.AutoValue; @@ -183,16 +183,20 @@ public class JdbcIO { } public static DataSourceConfiguration create(DataSource dataSource) { - checkNotNull(dataSource, "dataSource"); - checkArgument(dataSource instanceof Serializable, "dataSource must be Serializable"); + checkArgument(dataSource != null, "DataSourceConfiguration.create(dataSource) called with " + + "null data source"); + checkArgument(dataSource instanceof Serializable, + "DataSourceConfiguration.create(dataSource) called with a dataSource not Serializable"); return new AutoValue_JdbcIO_DataSourceConfiguration.Builder() .setDataSource(dataSource) .build(); } public static DataSourceConfiguration create(String driverClassName, String url) { - checkNotNull(driverClassName, "driverClassName"); - checkNotNull(url, "url"); + checkArgument(driverClassName != null, + "DataSourceConfiguration.create(driverClassName, url) called with null driverClassName"); + checkArgument(url != null, + "DataSourceConfiguration.create(driverClassName, url) called with null url"); return new AutoValue_JdbcIO_DataSourceConfiguration.Builder() .setDriverClassName(driverClassName) .setUrl(url) @@ -263,27 +267,31 @@ public class JdbcIO { } public Read withDataSourceConfiguration(DataSourceConfiguration configuration) { - checkNotNull(configuration, "configuration"); + checkArgument(configuration != null, "JdbcIO.read().withDataSourceConfiguration" + + "(configuration) called with null configuration"); return toBuilder().setDataSourceConfiguration(configuration).build(); } public Read withQuery(String query) { - checkNotNull(query, "query"); + checkArgument(query != null, "JdbcIO.read().withQuery(query) called with null query"); return toBuilder().setQuery(query).build(); } public Read withStatementPrepator(StatementPreparator statementPreparator) { - checkNotNull(statementPreparator, "statementPreparator"); + checkArgument(statementPreparator != null, + "JdbcIO.read().withStatementPreparator(statementPreparator) called " + + "with null statementPreparator"); return toBuilder().setStatementPreparator(statementPreparator).build(); } public Read withRowMapper(RowMapper rowMapper) { - checkNotNull(rowMapper, "rowMapper"); + checkArgument(rowMapper != null, + "JdbcIO.read().withRowMapper(rowMapper) called with null rowMapper"); return toBuilder().setRowMapper(rowMapper).build(); } public Read withCoder(Coder coder) { - checkNotNull(coder, "coder"); + checkArgument(coder != null, "JdbcIO.read().withCoder(coder) called with null coder"); return toBuilder().setCoder(coder).build(); } @@ -314,10 +322,15 @@ public class JdbcIO { @Override public void validate(PBegin input) { - checkNotNull(getQuery(), "query"); - checkNotNull(getRowMapper(), "rowMapper"); - checkNotNull(getCoder(), "coder"); -
[1/2] incubator-beam git commit: [BEAM-1034] Clean up tmp area in tests
Repository: incubator-beam Updated Branches: refs/heads/master 26a30a22d -> 6d0c205a3 [BEAM-1034] Clean up tmp area in tests Project: http://git-wip-us.apache.org/repos/asf/incubator-beam/repo Commit: http://git-wip-us.apache.org/repos/asf/incubator-beam/commit/ef74e192 Tree: http://git-wip-us.apache.org/repos/asf/incubator-beam/tree/ef74e192 Diff: http://git-wip-us.apache.org/repos/asf/incubator-beam/diff/ef74e192 Branch: refs/heads/master Commit: ef74e192eaee79e4cb8c7c901a296dd76559d76d Parents: 26a30a2 Author: Daniel KulpAuthored: Tue Nov 22 13:31:19 2016 -0500 Committer: Jean-Baptiste Onofré Committed: Thu Nov 24 07:55:58 2016 +0100 -- .../sorter/BufferedExternalSorter.java | 6 +- .../sdk/extensions/sorter/ExternalSorter.java | 6 +- .../sorter/BufferedExternalSorterTest.java | 58 +--- .../extensions/sorter/ExternalSorterTest.java | 53 +++--- 4 files changed, 103 insertions(+), 20 deletions(-) -- http://git-wip-us.apache.org/repos/asf/incubator-beam/blob/ef74e192/sdks/java/extensions/sorter/src/main/java/org/apache/beam/sdk/extensions/sorter/BufferedExternalSorter.java -- diff --git a/sdks/java/extensions/sorter/src/main/java/org/apache/beam/sdk/extensions/sorter/BufferedExternalSorter.java b/sdks/java/extensions/sorter/src/main/java/org/apache/beam/sdk/extensions/sorter/BufferedExternalSorter.java index 0f89e30..1dfd339 100644 --- a/sdks/java/extensions/sorter/src/main/java/org/apache/beam/sdk/extensions/sorter/BufferedExternalSorter.java +++ b/sdks/java/extensions/sorter/src/main/java/org/apache/beam/sdk/extensions/sorter/BufferedExternalSorter.java @@ -35,12 +35,13 @@ public class BufferedExternalSorter implements Sorter { private int memoryMB = 100; /** Sets the path to a temporary location where the sorter writes intermediate files. */ -public void setTempLocation(String tempLocation) { +public Options setTempLocation(String tempLocation) { checkArgument( !tempLocation.startsWith("gs://"), "BufferedExternalSorter does not support GCS temporary location"); this.tempLocation = tempLocation; + return this; } /** Returns the configured temporary location. */ @@ -52,9 +53,10 @@ public class BufferedExternalSorter implements Sorter { * Sets the size of the memory buffer in megabytes. This controls both the buffer for initial in * memory sorting and the buffer used when external sorting. Must be greater than zero. */ -public void setMemoryMB(int memoryMB) { +public Options setMemoryMB(int memoryMB) { checkArgument(memoryMB > 0, "memoryMB must be greater than zero"); this.memoryMB = memoryMB; + return this; } /** Returns the configured size of the memory buffer. */ http://git-wip-us.apache.org/repos/asf/incubator-beam/blob/ef74e192/sdks/java/extensions/sorter/src/main/java/org/apache/beam/sdk/extensions/sorter/ExternalSorter.java -- diff --git a/sdks/java/extensions/sorter/src/main/java/org/apache/beam/sdk/extensions/sorter/ExternalSorter.java b/sdks/java/extensions/sorter/src/main/java/org/apache/beam/sdk/extensions/sorter/ExternalSorter.java index 3cf0cc0..beef1ee 100644 --- a/sdks/java/extensions/sorter/src/main/java/org/apache/beam/sdk/extensions/sorter/ExternalSorter.java +++ b/sdks/java/extensions/sorter/src/main/java/org/apache/beam/sdk/extensions/sorter/ExternalSorter.java @@ -67,12 +67,13 @@ class ExternalSorter implements Sorter { private int memoryMB = 100; /** Sets the path to a temporary location where the sorter writes intermediate files. */ -public void setTempLocation(String tempLocation) { +public Options setTempLocation(String tempLocation) { if (tempLocation.startsWith("gs://")) { throw new IllegalArgumentException("Sorter doesn't support GCS temporary location."); } this.tempLocation = tempLocation; + return this; } /** Returns the configured temporary location. */ @@ -81,9 +82,10 @@ class ExternalSorter implements Sorter { } /** Sets the size of the memory buffer in megabytes. */ -public void setMemoryMB(int memoryMB) { +public Options setMemoryMB(int memoryMB) { checkArgument(memoryMB > 0, "memoryMB must be greater than zero"); this.memoryMB = memoryMB; + return this; } /** Returns the configured size of the memory buffer. */ http://git-wip-us.apache.org/repos/asf/incubator-beam/blob/ef74e192/sdks/java/extensions/sorter/src/test/java/org/apache/beam/sdk/extensions/sorter/BufferedExternalSorterTest.java
[2/2] incubator-beam git commit: [BEAM-1034] This closes #1415
[BEAM-1034] This closes #1415 Project: http://git-wip-us.apache.org/repos/asf/incubator-beam/repo Commit: http://git-wip-us.apache.org/repos/asf/incubator-beam/commit/6d0c205a Tree: http://git-wip-us.apache.org/repos/asf/incubator-beam/tree/6d0c205a Diff: http://git-wip-us.apache.org/repos/asf/incubator-beam/diff/6d0c205a Branch: refs/heads/master Commit: 6d0c205a306d6cdca346fe2aaf662b03b4959a0e Parents: 26a30a2 ef74e19 Author: Jean-Baptiste OnofréAuthored: Thu Nov 24 08:43:09 2016 +0100 Committer: Jean-Baptiste Onofré Committed: Thu Nov 24 08:43:09 2016 +0100 -- .../sorter/BufferedExternalSorter.java | 6 +- .../sdk/extensions/sorter/ExternalSorter.java | 6 +- .../sorter/BufferedExternalSorterTest.java | 58 +--- .../extensions/sorter/ExternalSorterTest.java | 53 +++--- 4 files changed, 103 insertions(+), 20 deletions(-) --
[1/2] incubator-beam git commit: [BEAM-1018] Updated getEstimatedSizeBytes() to use Number.longValue() in MongoDbIO
Repository: incubator-beam Updated Branches: refs/heads/master 875631f07 -> c6b86f60f [BEAM-1018] Updated getEstimatedSizeBytes() to use Number.longValue() in MongoDbIO Project: http://git-wip-us.apache.org/repos/asf/incubator-beam/repo Commit: http://git-wip-us.apache.org/repos/asf/incubator-beam/commit/a761b0c1 Tree: http://git-wip-us.apache.org/repos/asf/incubator-beam/tree/a761b0c1 Diff: http://git-wip-us.apache.org/repos/asf/incubator-beam/diff/a761b0c1 Branch: refs/heads/master Commit: a761b0c154b67daada2968c24c7de9c4afe13884 Parents: 875631f Author: Sandeep ParikhAuthored: Mon Nov 21 10:05:36 2016 -0600 Committer: Jean-Baptiste Onofré Committed: Mon Nov 21 20:37:11 2016 +0100 -- .../src/main/java/org/apache/beam/sdk/io/mongodb/MongoDbIO.java| 2 +- 1 file changed, 1 insertion(+), 1 deletion(-) -- http://git-wip-us.apache.org/repos/asf/incubator-beam/blob/a761b0c1/sdks/java/io/mongodb/src/main/java/org/apache/beam/sdk/io/mongodb/MongoDbIO.java -- diff --git a/sdks/java/io/mongodb/src/main/java/org/apache/beam/sdk/io/mongodb/MongoDbIO.java b/sdks/java/io/mongodb/src/main/java/org/apache/beam/sdk/io/mongodb/MongoDbIO.java index 71c017d..cd5cdaf 100644 --- a/sdks/java/io/mongodb/src/main/java/org/apache/beam/sdk/io/mongodb/MongoDbIO.java +++ b/sdks/java/io/mongodb/src/main/java/org/apache/beam/sdk/io/mongodb/MongoDbIO.java @@ -224,7 +224,7 @@ public class MongoDbIO { BasicDBObject stat = new BasicDBObject(); stat.append("collStats", spec.collection()); Document stats = mongoDatabase.runCommand(stat); - return Long.parseLong(stats.get("size").toString()); + return stats.get("size", Number.class).longValue(); } @Override
[2/2] incubator-beam git commit: [BEAM-1018] This closes #1394
[BEAM-1018] This closes #1394 Project: http://git-wip-us.apache.org/repos/asf/incubator-beam/repo Commit: http://git-wip-us.apache.org/repos/asf/incubator-beam/commit/c6b86f60 Tree: http://git-wip-us.apache.org/repos/asf/incubator-beam/tree/c6b86f60 Diff: http://git-wip-us.apache.org/repos/asf/incubator-beam/diff/c6b86f60 Branch: refs/heads/master Commit: c6b86f60f7b159fd188bfb6e5bf742a9b65080a0 Parents: 875631f a761b0c Author: Jean-Baptiste OnofréAuthored: Mon Nov 21 20:52:10 2016 +0100 Committer: Jean-Baptiste Onofré Committed: Mon Nov 21 20:52:10 2016 +0100 -- .../src/main/java/org/apache/beam/sdk/io/mongodb/MongoDbIO.java| 2 +- 1 file changed, 1 insertion(+), 1 deletion(-) --
[GitHub] incubator-beam pull request #1374: [BEAM-959] Improve validation messages in...
GitHub user jbonofre opened a pull request: https://github.com/apache/incubator-beam/pull/1374 [BEAM-959] Improve validation messages in JdbcIO Be sure to do all of the following to help us incorporate your contribution quickly and easily: - [X] Make sure the PR title is formatted like: `[BEAM-] Description of pull request` - [X] Make sure tests pass via `mvn clean verify`. (Even better, enable Travis-CI on your fork and ensure the whole test matrix passes). - [X] Replace `` in the title with the actual Jira issue number, if there is one. - [X] If this contribution is large, please file an Apache [Individual Contributor License Agreement](https://www.apache.org/licenses/icla.txt). --- You can merge this pull request into a Git repository by running: $ git pull https://github.com/jbonofre/incubator-beam BEAM-959 Alternatively you can review and apply these changes as the patch at: https://github.com/apache/incubator-beam/pull/1374.patch To close this pull request, make a commit to your master/trunk branch with (at least) the following in the commit message: This closes #1374 commit cd5336b7cd66a283ca705684d6d0217a6938d77f Author: Jean-Baptiste Onofré <jbono...@apache.org> Date: 2016-11-17T16:07:21Z [BEAM-959] Improve validation messages in JdbcIO --- If your project is set up for it, you can reply to this email and have your reply appear on GitHub as well. If your project does not have this feature enabled and wishes so, or if the feature is enabled but not working, please contact infrastructure at infrastruct...@apache.org or file a JIRA ticket with INFRA. ---
[GitHub] incubator-beam pull request #1361: [BEAM-930] Fix findbugs and re-enable Mav...
GitHub user jbonofre opened a pull request: https://github.com/apache/incubator-beam/pull/1361 [BEAM-930] Fix findbugs and re-enable Maven plugin in MongoDbIO and MongoDbGridFSIO Be sure to do all of the following to help us incorporate your contribution quickly and easily: - [X] Make sure the PR title is formatted like: `[BEAM-] Description of pull request` - [X] Make sure tests pass via `mvn clean verify`. (Even better, enable Travis-CI on your fork and ensure the whole test matrix passes). - [X] Replace `` in the title with the actual Jira issue number, if there is one. - [X] If this contribution is large, please file an Apache [Individual Contributor License Agreement](https://www.apache.org/licenses/icla.txt). --- You can merge this pull request into a Git repository by running: $ git pull https://github.com/jbonofre/incubator-beam BEAM-930 Alternatively you can review and apply these changes as the patch at: https://github.com/apache/incubator-beam/pull/1361.patch To close this pull request, make a commit to your master/trunk branch with (at least) the following in the commit message: This closes #1361 --- If your project is set up for it, you can reply to this email and have your reply appear on GitHub as well. If your project does not have this feature enabled and wishes so, or if the feature is enabled but not working, please contact infrastructure at infrastruct...@apache.org or file a JIRA ticket with INFRA. ---
[2/2] incubator-beam git commit: [BEAM-948] This closes #1324
[BEAM-948] This closes #1324 Project: http://git-wip-us.apache.org/repos/asf/incubator-beam/repo Commit: http://git-wip-us.apache.org/repos/asf/incubator-beam/commit/9c300cde Tree: http://git-wip-us.apache.org/repos/asf/incubator-beam/tree/9c300cde Diff: http://git-wip-us.apache.org/repos/asf/incubator-beam/diff/9c300cde Branch: refs/heads/master Commit: 9c300cde8cca3e2c0ccdf9e87cbf4946c4199517 Parents: 13ad8f6 79d5ad9 Author: Jean-Baptiste OnofréAuthored: Tue Nov 15 12:05:20 2016 +0100 Committer: Jean-Baptiste Onofré Committed: Tue Nov 15 12:05:20 2016 +0100 -- .../beam/sdk/io/mongodb/MongoDbGridFSIO.java| 308 +-- .../sdk/io/mongodb/MongoDBGridFSIOTest.java | 93 ++ 2 files changed, 369 insertions(+), 32 deletions(-) --
[1/2] incubator-beam git commit: [BEAM-948] Add ability to write files to GridFS
Repository: incubator-beam Updated Branches: refs/heads/master 13ad8f68a -> 9c300cde8 [BEAM-948] Add ability to write files to GridFS Project: http://git-wip-us.apache.org/repos/asf/incubator-beam/repo Commit: http://git-wip-us.apache.org/repos/asf/incubator-beam/commit/79d5ad9d Tree: http://git-wip-us.apache.org/repos/asf/incubator-beam/tree/79d5ad9d Diff: http://git-wip-us.apache.org/repos/asf/incubator-beam/diff/79d5ad9d Branch: refs/heads/master Commit: 79d5ad9d7f629481cf04dd65e83f1b06708e16bc Parents: 13ad8f6 Author: Daniel KulpAuthored: Mon Nov 7 17:21:55 2016 -0500 Committer: Jean-Baptiste Onofré Committed: Tue Nov 15 11:46:34 2016 +0100 -- .../beam/sdk/io/mongodb/MongoDbGridFSIO.java| 308 +-- .../sdk/io/mongodb/MongoDBGridFSIOTest.java | 93 ++ 2 files changed, 369 insertions(+), 32 deletions(-) -- http://git-wip-us.apache.org/repos/asf/incubator-beam/blob/79d5ad9d/sdks/java/io/mongodb/src/main/java/org/apache/beam/sdk/io/mongodb/MongoDbGridFSIO.java -- diff --git a/sdks/java/io/mongodb/src/main/java/org/apache/beam/sdk/io/mongodb/MongoDbGridFSIO.java b/sdks/java/io/mongodb/src/main/java/org/apache/beam/sdk/io/mongodb/MongoDbGridFSIO.java index 8c9a65c..26e2c2f 100644 --- a/sdks/java/io/mongodb/src/main/java/org/apache/beam/sdk/io/mongodb/MongoDbGridFSIO.java +++ b/sdks/java/io/mongodb/src/main/java/org/apache/beam/sdk/io/mongodb/MongoDbGridFSIO.java @@ -17,6 +17,7 @@ */ package org.apache.beam.sdk.io.mongodb; +import static com.google.common.base.Preconditions.checkArgument; import static com.google.common.base.Preconditions.checkNotNull; import com.google.auto.value.AutoValue; @@ -27,11 +28,13 @@ import com.mongodb.Mongo; import com.mongodb.MongoURI; import com.mongodb.gridfs.GridFS; import com.mongodb.gridfs.GridFSDBFile; +import com.mongodb.gridfs.GridFSInputFile; import com.mongodb.util.JSON; import java.io.BufferedReader; import java.io.IOException; import java.io.InputStreamReader; +import java.io.OutputStream; import java.io.Serializable; import java.util.ArrayList; import java.util.Iterator; @@ -51,6 +54,7 @@ import org.apache.beam.sdk.transforms.ParDo; import org.apache.beam.sdk.transforms.display.DisplayData; import org.apache.beam.sdk.values.PBegin; import org.apache.beam.sdk.values.PCollection; +import org.apache.beam.sdk.values.PDone; import org.bson.types.ObjectId; import org.joda.time.Duration; import org.joda.time.Instant; @@ -84,6 +88,36 @@ import org.joda.time.Instant; * the file as the timestamp. * When using a parser that outputs with custom timestamps, you may also need to specify * the allowedTimestampSkew option. + * + * + * + * Writing to MongoDB via GridFS + * + * MongoDBGridFS supports writing of data to a file in a MongoDB GridFS collection. + * + * To configure a MongoDB GridFS sink, you can provide the connection URI, the database name + * and the bucket name. You must also provide the filename to write to. Another optional parameter + * is the GridFS file chunkSize. + * + * For instance: + * + * {@code + * + * pipeline + * .apply(...) + * .apply(MongoDbGridFSIO.write() + * .withUri("mongodb://localhost:27017") + * .withDatabase("my-database") + * .withBucket("my-bucket") + * .withChunkSize(256000L) + * .withFilename("my-output.txt")) + * + * } + * + * There is also an optional argument to the {@code create()} method to specify a writer + * that is used to write the data to the OutputStream. By default, it writes UTF-8 strings + * to the file separated with line feeds. + * */ public class MongoDbGridFSIO { @@ -136,19 +170,68 @@ public class MongoDbGridFSIO { /** Read data from GridFS. Default behavior with String. */ public static Read read() { -return new AutoValue_MongoDbGridFSIO_Read.Builder().build() -.withParser(TEXT_PARSER).withCoder(StringUtf8Coder.of()); +return new AutoValue_MongoDbGridFSIO_Read.Builder() +.setParser(TEXT_PARSER) +.setCoder(StringUtf8Coder.of()) +.setConnectionConfiguration(ConnectionConfiguration.create()) +.build(); } + /** Write data to GridFS. Default behavior with String. */ + public static Write write() { +return new AutoValue_MongoDbGridFSIO_Write.Builder() +.setConnectionConfiguration(ConnectionConfiguration.create()) +.setWriteFn(new WriteFn() { + @Override + public void write(String output, OutputStream outStream) throws IOException { +outStream.write(output.getBytes("utf-8")); +outStream.write('\n'); + } +}).build(); + } + public static Write write(WriteFn fn) { +return new AutoValue_MongoDbGridFSIO_Write.Builder() +
[GitHub] incubator-beam pull request #1357: [BEAM-927] Fix findbugs and re-enable Mav...
GitHub user jbonofre opened a pull request: https://github.com/apache/incubator-beam/pull/1357 [BEAM-927] Fix findbugs and re-enable Maven plugin in JmsIO Be sure to do all of the following to help us incorporate your contribution quickly and easily: - [X] Make sure the PR title is formatted like: `[BEAM-] Description of pull request` - [X] Make sure tests pass via `mvn clean verify`. (Even better, enable Travis-CI on your fork and ensure the whole test matrix passes). - [X] Replace `` in the title with the actual Jira issue number, if there is one. - [X] If this contribution is large, please file an Apache [Individual Contributor License Agreement](https://www.apache.org/licenses/icla.txt). --- You can merge this pull request into a Git repository by running: $ git pull https://github.com/jbonofre/incubator-beam BEAM-927 Alternatively you can review and apply these changes as the patch at: https://github.com/apache/incubator-beam/pull/1357.patch To close this pull request, make a commit to your master/trunk branch with (at least) the following in the commit message: This closes #1357 commit 5cd00e5bd43b47b3cb7296001344310ca6f97305 Author: Jean-Baptiste Onofré <jbono...@apache.org> Date: 2016-11-14T15:27:56Z [BEAM-927] Fix findbugs and re-enable Maven plugin in JmsIO --- If your project is set up for it, you can reply to this email and have your reply appear on GitHub as well. If your project does not have this feature enabled and wishes so, or if the feature is enabled but not working, please contact infrastructure at infrastruct...@apache.org or file a JIRA ticket with INFRA. ---
[GitHub] incubator-beam pull request #1356: [BEAM-930] Fix findbugs and re-enable Mav...
GitHub user jbonofre opened a pull request: https://github.com/apache/incubator-beam/pull/1356 [BEAM-930] Fix findbugs and re-enable Maven plugin in MongoDbIO Be sure to do all of the following to help us incorporate your contribution quickly and easily: - [X] Make sure the PR title is formatted like: `[BEAM-] Description of pull request` - [X] Make sure tests pass via `mvn clean verify`. (Even better, enable Travis-CI on your fork and ensure the whole test matrix passes). - [X] Replace `` in the title with the actual Jira issue number, if there is one. - [X] If this contribution is large, please file an Apache [Individual Contributor License Agreement](https://www.apache.org/licenses/icla.txt). --- You can merge this pull request into a Git repository by running: $ git pull https://github.com/jbonofre/incubator-beam BEAM-930 Alternatively you can review and apply these changes as the patch at: https://github.com/apache/incubator-beam/pull/1356.patch To close this pull request, make a commit to your master/trunk branch with (at least) the following in the commit message: This closes #1356 commit d838d95d62fec12e7cd990c00bdaaedc6b7ad6ea Author: Jean-Baptiste Onofré <jbono...@apache.org> Date: 2016-11-14T15:10:53Z [BEAM-930] Fix findbugs and re-enable Maven plugin in MongoDbIO --- If your project is set up for it, you can reply to this email and have your reply appear on GitHub as well. If your project does not have this feature enabled and wishes so, or if the feature is enabled but not working, please contact infrastructure at infrastruct...@apache.org or file a JIRA ticket with INFRA. ---
[GitHub] incubator-beam-site pull request #64: [BEAM-501] Update on the v2 skin propo...
GitHub user jbonofre opened a pull request: https://github.com/apache/incubator-beam-site/pull/64 [BEAM-501] Update on the v2 skin proposal This PR contains the updated v2 skin proposal. I "migrated" only couple of page to show the rendering. I will add new commits in the PR after review on the first pages. You can merge this pull request into a Git repository by running: $ git pull https://github.com/jbonofre/incubator-beam-site mockup Alternatively you can review and apply these changes as the patch at: https://github.com/apache/incubator-beam-site/pull/64.patch To close this pull request, make a commit to your master/trunk branch with (at least) the following in the commit message: This closes #64 commit 0dddc65db4fcc4e333555f307699d7a921f1b04c Author: Jean-Baptiste Onofré <jbono...@apache.org> Date: 2016-11-03T14:32:28Z [BEAM-501] Update on the new v2 skin proposal --- If your project is set up for it, you can reply to this email and have your reply appear on GitHub as well. If your project does not have this feature enabled and wishes so, or if the feature is enabled but not working, please contact infrastructure at infrastruct...@apache.org or file a JIRA ticket with INFRA. ---
[GitHub] incubator-beam pull request #1262: [BEAM-856] Use free network port for the ...
GitHub user jbonofre opened a pull request: https://github.com/apache/incubator-beam/pull/1262 [BEAM-856] Use free network port for the Derby test instance Be sure to do all of the following to help us incorporate your contribution quickly and easily: - [X] Make sure the PR title is formatted like: `[BEAM-] Description of pull request` - [X] Make sure tests pass via `mvn clean verify`. (Even better, enable Travis-CI on your fork and ensure the whole test matrix passes). - [X] Replace `` in the title with the actual Jira issue number, if there is one. - [X] If this contribution is large, please file an Apache [Individual Contributor License Agreement](https://www.apache.org/licenses/icla.txt). --- You can merge this pull request into a Git repository by running: $ git pull https://github.com/jbonofre/incubator-beam BEAM-856-JDBCIO Alternatively you can review and apply these changes as the patch at: https://github.com/apache/incubator-beam/pull/1262.patch To close this pull request, make a commit to your master/trunk branch with (at least) the following in the commit message: This closes #1262 commit 9dcefd72e5629ea6bb61f29f99ca08f0e52ca7d3 Author: Jean-Baptiste Onofré <jbono...@apache.org> Date: 2016-11-02T18:22:16Z [BEAM-856] Use free network port for the Derby test instance --- If your project is set up for it, you can reply to this email and have your reply appear on GitHub as well. If your project does not have this feature enabled and wishes so, or if the feature is enabled but not working, please contact infrastructure at infrastruct...@apache.org or file a JIRA ticket with INFRA. ---
[1/2] incubator-beam git commit: [BEAM-864] Update Apache POM parent to 18 and apache-rat-plugin to 0.12 to exclude DEPENDENCIES by default
Repository: incubator-beam Updated Branches: refs/heads/master 978c99e9d -> a38a6072d [BEAM-864] Update Apache POM parent to 18 and apache-rat-plugin to 0.12 to exclude DEPENDENCIES by default Project: http://git-wip-us.apache.org/repos/asf/incubator-beam/repo Commit: http://git-wip-us.apache.org/repos/asf/incubator-beam/commit/300f4112 Tree: http://git-wip-us.apache.org/repos/asf/incubator-beam/tree/300f4112 Diff: http://git-wip-us.apache.org/repos/asf/incubator-beam/diff/300f4112 Branch: refs/heads/master Commit: 300f4112fd04116a180ffe297a8a58029a6f61a5 Parents: 978c99e Author: Jean-Baptiste OnofréAuthored: Mon Oct 31 07:08:38 2016 +0100 Committer: Jean-Baptiste Onofré Committed: Tue Nov 1 07:53:24 2016 +0100 -- pom.xml | 4 ++-- 1 file changed, 2 insertions(+), 2 deletions(-) -- http://git-wip-us.apache.org/repos/asf/incubator-beam/blob/300f4112/pom.xml -- diff --git a/pom.xml b/pom.xml index 453e57c..2820732 100644 --- a/pom.xml +++ b/pom.xml @@ -22,7 +22,7 @@ org.apache apache -17 +18 org.apache.beam @@ -907,7 +907,7 @@ org.apache.rat apache-rat-plugin - 0.11 + 0.12 ${project.build.directory}/${project.build.finalName}.rat false
[2/2] incubator-beam git commit: [BEAM-864] This closes #1233
[BEAM-864] This closes #1233 Project: http://git-wip-us.apache.org/repos/asf/incubator-beam/repo Commit: http://git-wip-us.apache.org/repos/asf/incubator-beam/commit/a38a6072 Tree: http://git-wip-us.apache.org/repos/asf/incubator-beam/tree/a38a6072 Diff: http://git-wip-us.apache.org/repos/asf/incubator-beam/diff/a38a6072 Branch: refs/heads/master Commit: a38a6072d9a634f59e48aa55e03404a061c8a850 Parents: 978c99e 300f411 Author: Jean-Baptiste OnofréAuthored: Tue Nov 1 10:43:50 2016 +0100 Committer: Jean-Baptiste Onofré Committed: Tue Nov 1 10:43:50 2016 +0100 -- pom.xml | 4 ++-- 1 file changed, 2 insertions(+), 2 deletions(-) --
[GitHub] incubator-beam pull request #1236: [BEAM-856] Use free available port to sta...
GitHub user jbonofre opened a pull request: https://github.com/apache/incubator-beam/pull/1236 [BEAM-856] Use free available port to start the test MongoDB instance Be sure to do all of the following to help us incorporate your contribution quickly and easily: - [X] Make sure the PR title is formatted like: `[BEAM-] Description of pull request` - [X] Make sure tests pass via `mvn clean verify`. (Even better, enable Travis-CI on your fork and ensure the whole test matrix passes). - [X] Replace `` in the title with the actual Jira issue number, if there is one. - [X] If this contribution is large, please file an Apache [Individual Contributor License Agreement](https://www.apache.org/licenses/icla.txt). --- When agreed on this PR, I will apply the same logic for the other IOs (JdbcIO, JmsIO, ...). You can merge this pull request into a Git repository by running: $ git pull https://github.com/jbonofre/incubator-beam BEAM-856 Alternatively you can review and apply these changes as the patch at: https://github.com/apache/incubator-beam/pull/1236.patch To close this pull request, make a commit to your master/trunk branch with (at least) the following in the commit message: This closes #1236 commit 3f8a6cf6e7a3b0cce90b8ab033d675d8e1fdca2c Author: Jean-Baptiste Onofré <jbono...@apache.org> Date: 2016-10-31T16:27:15Z [BEAM-856] Use free available port to start the test MongoDB instance --- If your project is set up for it, you can reply to this email and have your reply appear on GitHub as well. If your project does not have this feature enabled and wishes so, or if the feature is enabled but not working, please contact infrastructure at infrastruct...@apache.org or file a JIRA ticket with INFRA. ---
[1/2] incubator-beam git commit: [BEAM-871] Add StatementPreparator on JdbcIO
Repository: incubator-beam Updated Branches: refs/heads/master 594892d11 -> 54a737402 [BEAM-871] Add StatementPreparator on JdbcIO Project: http://git-wip-us.apache.org/repos/asf/incubator-beam/repo Commit: http://git-wip-us.apache.org/repos/asf/incubator-beam/commit/365b627e Tree: http://git-wip-us.apache.org/repos/asf/incubator-beam/tree/365b627e Diff: http://git-wip-us.apache.org/repos/asf/incubator-beam/diff/365b627e Branch: refs/heads/master Commit: 365b627ea0dfdc3e5d4cd3f0fe98c0ffb502e3be Parents: 594892d Author: Gareth WesternAuthored: Mon Oct 3 00:07:15 2016 +0200 Committer: Jean-Baptiste Onofré Committed: Mon Oct 31 16:01:01 2016 +0100 -- .../org/apache/beam/sdk/io/jdbc/JdbcIO.java | 39 .../org/apache/beam/sdk/io/jdbc/JdbcIOTest.java | 33 + 2 files changed, 72 insertions(+) -- http://git-wip-us.apache.org/repos/asf/incubator-beam/blob/365b627e/sdks/java/io/jdbc/src/main/java/org/apache/beam/sdk/io/jdbc/JdbcIO.java -- diff --git a/sdks/java/io/jdbc/src/main/java/org/apache/beam/sdk/io/jdbc/JdbcIO.java b/sdks/java/io/jdbc/src/main/java/org/apache/beam/sdk/io/jdbc/JdbcIO.java index 505cdee..0e0703f 100644 --- a/sdks/java/io/jdbc/src/main/java/org/apache/beam/sdk/io/jdbc/JdbcIO.java +++ b/sdks/java/io/jdbc/src/main/java/org/apache/beam/sdk/io/jdbc/JdbcIO.java @@ -77,6 +77,27 @@ import org.apache.commons.dbcp2.BasicDataSource; * }) * } * + * Query parameters can be configured using a user-provided {@link StatementPreparator}. + * For example: + * + * {@code + * pipeline.apply(JdbcIO. >read() + * .withDataSourceConfiguration(JdbcIO.DataSourceConfiguration.create( + * "com.mysql.jdbc.Driver", "jdbc:mysql://hostname:3306/mydb", + * "username", "password")) + * .withQuery("select id,name from Person where name = ?") + * .withStatementPreparator(new JdbcIO.StatementPreparator() { + * public void setParameters(PreparedStatement preparedStatement) throws Exception { + * preparedStatement.setString(1, "Darwin"); + * } + * }) + * .withRowMapper(new JdbcIO.RowMapper >() { + * public KV mapRow(ResultSet resultSet) throws Exception { + * return KV.of(resultSet.getInt(1), resultSet.getString(2)); + * } + * }) + * } + * * Writing to JDBC datasource * * JDBC sink supports writing records into a database. It writes a {@link PCollection} to the @@ -212,11 +233,20 @@ public class JdbcIO { } } + /** + * An interface used by the JdbcIO Write to set the parameters of the {@link PreparedStatement} + * used to setParameters into the database. + */ + public interface StatementPreparator extends Serializable { +void setParameters(PreparedStatement preparedStatement) throws Exception; + } + /** A {@link PTransform} to read data from a JDBC datasource. */ @AutoValue public abstract static class Read extends PTransform { @Nullable abstract DataSourceConfiguration getDataSourceConfiguration(); @Nullable abstract String getQuery(); +@Nullable abstract StatementPreparator getStatementPreparator(); @Nullable abstract RowMapper getRowMapper(); @Nullable abstract Coder getCoder(); @@ -226,6 +256,7 @@ public class JdbcIO { abstract static class Builder { abstract Builder setDataSourceConfiguration(DataSourceConfiguration config); abstract Builder setQuery(String query); + abstract Builder setStatementPreparator(StatementPreparator statementPreparator); abstract Builder setRowMapper(RowMapper rowMapper); abstract Builder setCoder(Coder coder); abstract Read build(); @@ -241,6 +272,11 @@ public class JdbcIO { return toBuilder().setQuery(query).build(); } +public Read withStatementPrepator(StatementPreparator statementPreparator) { + checkNotNull(statementPreparator, "statementPreparator"); + return toBuilder().setStatementPreparator(statementPreparator).build(); +} + public Read withRowMapper(RowMapper rowMapper) { checkNotNull(rowMapper, "rowMapper"); return toBuilder().setRowMapper(rowMapper).build(); @@ -311,6 +347,9 @@ public class JdbcIO { public void processElement(ProcessContext context) throws Exception { String query = context.element(); try (PreparedStatement statement = connection.prepareStatement(query)) { + if (this.spec.getStatementPreparator() != null) { +this.spec.getStatementPreparator().setParameters(statement); + } try (ResultSet resultSet = statement.executeQuery()) { while (resultSet.next()) {
[2/2] incubator-beam git commit: [BEAM-871] This closes #1037
[BEAM-871] This closes #1037 Project: http://git-wip-us.apache.org/repos/asf/incubator-beam/repo Commit: http://git-wip-us.apache.org/repos/asf/incubator-beam/commit/54a73740 Tree: http://git-wip-us.apache.org/repos/asf/incubator-beam/tree/54a73740 Diff: http://git-wip-us.apache.org/repos/asf/incubator-beam/diff/54a73740 Branch: refs/heads/master Commit: 54a737402b61c5a5a707fabde55affaea716eca2 Parents: 594892d 365b627 Author: Jean-Baptiste OnofréAuthored: Mon Oct 31 16:14:57 2016 +0100 Committer: Jean-Baptiste Onofré Committed: Mon Oct 31 16:14:57 2016 +0100 -- .../org/apache/beam/sdk/io/jdbc/JdbcIO.java | 39 .../org/apache/beam/sdk/io/jdbc/JdbcIOTest.java | 33 + 2 files changed, 72 insertions(+) --
[GitHub] incubator-beam pull request #1233: [BEAM-864] Update Apaache POM parent to 1...
GitHub user jbonofre opened a pull request: https://github.com/apache/incubator-beam/pull/1233 [BEAM-864] Update Apaache POM parent to 18 and apache-rat-plugin to 0.12 to exclude DEPENDENCIES by default Be sure to do all of the following to help us incorporate your contribution quickly and easily: - [ ] Make sure the PR title is formatted like: `[BEAM-] Description of pull request` - [ ] Make sure tests pass via `mvn clean verify`. (Even better, enable Travis-CI on your fork and ensure the whole test matrix passes). - [ ] Replace `` in the title with the actual Jira issue number, if there is one. - [ ] If this contribution is large, please file an Apache [Individual Contributor License Agreement](https://www.apache.org/licenses/icla.txt). --- You can merge this pull request into a Git repository by running: $ git pull https://github.com/jbonofre/incubator-beam BEAM-864 Alternatively you can review and apply these changes as the patch at: https://github.com/apache/incubator-beam/pull/1233.patch To close this pull request, make a commit to your master/trunk branch with (at least) the following in the commit message: This closes #1233 commit 735983448c5ad9a225e2b12595ae679c2c23a982 Author: Jean-Baptiste Onofré <jbono...@apache.org> Date: 2016-10-31T06:08:38Z [BEAM-864] Update Apaache POM parent to 18 and apache-rat-plugin to 0.12 to exclude DEPENDENCIES by default --- If your project is set up for it, you can reply to this email and have your reply appear on GitHub as well. If your project does not have this feature enabled and wishes so, or if the feature is enabled but not working, please contact infrastructure at infrastruct...@apache.org or file a JIRA ticket with INFRA. ---
[GitHub] incubator-beam-site pull request #51: [BEAM-501] New skin proposal
GitHub user jbonofre opened a pull request: https://github.com/apache/incubator-beam-site/pull/51 [BEAM-501] New skin proposal This a new skin proposal. It's still a work in progress, but it already allows people to take a look, provide feedback, and upgrade. You can merge this pull request into a Git repository by running: $ git pull https://github.com/jbonofre/incubator-beam-site mockup Alternatively you can review and apply these changes as the patch at: https://github.com/apache/incubator-beam-site/pull/51.patch To close this pull request, make a commit to your master/trunk branch with (at least) the following in the commit message: This closes #51 commit 9d389bbf2be843a3b2ca45951636e26d505b3207 Author: Jean-Baptiste Onofré <jbono...@apache.org> Date: 2016-10-20T08:38:10Z [BEAM-501] New skin proposal --- If your project is set up for it, you can reply to this email and have your reply appear on GitHub as well. If your project does not have this feature enabled and wishes so, or if the feature is enabled but not working, please contact infrastructure at infrastruct...@apache.org or file a JIRA ticket with INFRA. ---
[2/2] incubator-beam git commit: [BEAM-743] This closes #1084
[BEAM-743] This closes #1084 Project: http://git-wip-us.apache.org/repos/asf/incubator-beam/repo Commit: http://git-wip-us.apache.org/repos/asf/incubator-beam/commit/dde8e35c Tree: http://git-wip-us.apache.org/repos/asf/incubator-beam/tree/dde8e35c Diff: http://git-wip-us.apache.org/repos/asf/incubator-beam/diff/dde8e35c Branch: refs/heads/master Commit: dde8e35ca88f14fcf93349e20c5e70f991308b1e Parents: 3b1c2a3 1cb6200 Author: Jean-Baptiste OnofréAuthored: Wed Oct 19 08:13:23 2016 +0200 Committer: Jean-Baptiste Onofré Committed: Wed Oct 19 08:13:23 2016 +0200 -- .../org/apache/beam/sdk/io/jdbc/JdbcIO.java | 70 .../org/apache/beam/sdk/io/jdbc/JdbcIOTest.java | 39 ++- 2 files changed, 80 insertions(+), 29 deletions(-) --
[1/2] incubator-beam git commit: Build in eclipse/eclipse-jdt
Repository: incubator-beam Updated Branches: refs/heads/master c5329f9b4 -> a2c342cfd Build in eclipse/eclipse-jdt Project: http://git-wip-us.apache.org/repos/asf/incubator-beam/repo Commit: http://git-wip-us.apache.org/repos/asf/incubator-beam/commit/957c545e Tree: http://git-wip-us.apache.org/repos/asf/incubator-beam/tree/957c545e Diff: http://git-wip-us.apache.org/repos/asf/incubator-beam/diff/957c545e Branch: refs/heads/master Commit: 957c545eaa33c861b561418b1c7dadf4c31f92f3 Parents: c5329f9 Author: Daniel KulpAuthored: Thu Oct 13 12:41:32 2016 -0400 Committer: Jean-Baptiste Onofré Committed: Tue Oct 18 16:53:10 2016 +0200 -- .travis.yml | 2 ++ .../beam/runners/direct/BoundedReadEvaluatorFactory.java | 3 ++- .../java/org/apache/beam/runners/direct/DirectMetrics.java| 7 +++ .../beam/runners/direct/TestStreamEvaluatorFactory.java | 2 +- .../beam/runners/direct/UnboundedReadEvaluatorFactory.java| 5 +++-- .../main/java/org/apache/beam/runners/spark/io/SourceRDD.java | 6 -- 6 files changed, 19 insertions(+), 6 deletions(-) -- http://git-wip-us.apache.org/repos/asf/incubator-beam/blob/957c545e/.travis.yml -- diff --git a/.travis.yml b/.travis.yml index 786b370..5133a43 100644 --- a/.travis.yml +++ b/.travis.yml @@ -39,6 +39,8 @@ matrix: - os: linux env: CUSTOM_JDK="oraclejdk7" MAVEN_OVERRIDE="-DbeamSurefireArgline='-Xmx512m'" - os: linux + env: CUSTOM_JDK="oraclejdk7" MAVEN_OVERRIDE="-DbeamSurefireArgline='-Xmx512m' -Peclipse-jdt" +- os: linux env: CUSTOM_JDK="openjdk7" MAVEN_OVERRIDE="-DbeamSurefireArgline='-Xmx512m'" before_install: http://git-wip-us.apache.org/repos/asf/incubator-beam/blob/957c545e/runners/direct-java/src/main/java/org/apache/beam/runners/direct/BoundedReadEvaluatorFactory.java -- diff --git a/runners/direct-java/src/main/java/org/apache/beam/runners/direct/BoundedReadEvaluatorFactory.java b/runners/direct-java/src/main/java/org/apache/beam/runners/direct/BoundedReadEvaluatorFactory.java index 843dcd6..add1e8a 100644 --- a/runners/direct-java/src/main/java/org/apache/beam/runners/direct/BoundedReadEvaluatorFactory.java +++ b/runners/direct-java/src/main/java/org/apache/beam/runners/direct/BoundedReadEvaluatorFactory.java @@ -34,6 +34,7 @@ import org.apache.beam.sdk.transforms.AppliedPTransform; import org.apache.beam.sdk.transforms.PTransform; import org.apache.beam.sdk.transforms.windowing.BoundedWindow; import org.apache.beam.sdk.util.WindowedValue; +import org.apache.beam.sdk.values.PBegin; import org.apache.beam.sdk.values.PCollection; import org.slf4j.Logger; import org.slf4j.LoggerFactory; @@ -137,7 +138,7 @@ final class BoundedReadEvaluatorFactory implements TransformEvaluatorFactory { private Collection > createInitialSplits( -AppliedPTransform> transform, int targetParallelism) +AppliedPTransform transform, int targetParallelism) throws Exception { BoundedSource source = transform.getTransform().getSource(); PipelineOptions options = evaluationContext.getPipelineOptions(); http://git-wip-us.apache.org/repos/asf/incubator-beam/blob/957c545e/runners/direct-java/src/main/java/org/apache/beam/runners/direct/DirectMetrics.java -- diff --git a/runners/direct-java/src/main/java/org/apache/beam/runners/direct/DirectMetrics.java b/runners/direct-java/src/main/java/org/apache/beam/runners/direct/DirectMetrics.java index a749a76..145326f 100644 --- a/runners/direct-java/src/main/java/org/apache/beam/runners/direct/DirectMetrics.java +++ b/runners/direct-java/src/main/java/org/apache/beam/runners/direct/DirectMetrics.java @@ -224,6 +224,13 @@ class DirectMetrics extends MetricResults { @AutoValue abstract static class DirectMetricResult implements MetricResult { +// need to define these here so they appear in the correct order +// and the generated constructor is usable and consistent +public abstract MetricName name(); +public abstract String step(); +public abstract T committed(); +public abstract T attempted(); + public static MetricResult create(MetricName name, String scope, T committed, T attempted) { return new AutoValue_DirectMetrics_DirectMetricResult( http://git-wip-us.apache.org/repos/asf/incubator-beam/blob/957c545e/runners/direct-java/src/main/java/org/apache/beam/runners/direct/TestStreamEvaluatorFactory.java
[2/2] incubator-beam git commit: This closes #1094
This closes #1094 Project: http://git-wip-us.apache.org/repos/asf/incubator-beam/repo Commit: http://git-wip-us.apache.org/repos/asf/incubator-beam/commit/a2c342cf Tree: http://git-wip-us.apache.org/repos/asf/incubator-beam/tree/a2c342cf Diff: http://git-wip-us.apache.org/repos/asf/incubator-beam/diff/a2c342cf Branch: refs/heads/master Commit: a2c342cfd2c2f86593e6d4f30c25b8054fa2ffa2 Parents: c5329f9 957c545 Author: Jean-Baptiste OnofréAuthored: Tue Oct 18 16:54:12 2016 +0200 Committer: Jean-Baptiste Onofré Committed: Tue Oct 18 16:54:12 2016 +0200 -- .travis.yml | 2 ++ .../beam/runners/direct/BoundedReadEvaluatorFactory.java | 3 ++- .../java/org/apache/beam/runners/direct/DirectMetrics.java| 7 +++ .../beam/runners/direct/TestStreamEvaluatorFactory.java | 2 +- .../beam/runners/direct/UnboundedReadEvaluatorFactory.java| 5 +++-- .../main/java/org/apache/beam/runners/spark/io/SourceRDD.java | 6 -- 6 files changed, 19 insertions(+), 6 deletions(-) --
[2/2] incubator-beam git commit: [BEAM-740] This closes #1082
[BEAM-740] This closes #1082 Project: http://git-wip-us.apache.org/repos/asf/incubator-beam/repo Commit: http://git-wip-us.apache.org/repos/asf/incubator-beam/commit/1de6439f Tree: http://git-wip-us.apache.org/repos/asf/incubator-beam/tree/1de6439f Diff: http://git-wip-us.apache.org/repos/asf/incubator-beam/diff/1de6439f Branch: refs/heads/master Commit: 1de6439f14f3a1716427f10606d89a29413367d9 Parents: 6c88216 8017a02 Author: Jean-Baptiste OnofréAuthored: Tue Oct 11 09:57:57 2016 +0200 Committer: Jean-Baptiste Onofré Committed: Tue Oct 11 09:57:57 2016 +0200 -- runners/flink/README.md | 10 +- 1 file changed, 9 insertions(+), 1 deletion(-) --
[1/2] incubator-beam git commit: [BEAM-740] improve runners-flink README.md
Repository: incubator-beam Updated Branches: refs/heads/master 6c8821682 -> 1de6439f1 [BEAM-740] improve runners-flink README.md Project: http://git-wip-us.apache.org/repos/asf/incubator-beam/repo Commit: http://git-wip-us.apache.org/repos/asf/incubator-beam/commit/8017a025 Tree: http://git-wip-us.apache.org/repos/asf/incubator-beam/tree/8017a025 Diff: http://git-wip-us.apache.org/repos/asf/incubator-beam/diff/8017a025 Branch: refs/heads/master Commit: 8017a02597e260dd9e3306fba5165385e078c645 Parents: 6c88216 Author: manuzhangAuthored: Mon Oct 10 19:38:35 2016 +0800 Committer: Jean-Baptiste Onofré Committed: Tue Oct 11 09:03:16 2016 +0200 -- runners/flink/README.md | 10 +- 1 file changed, 9 insertions(+), 1 deletion(-) -- http://git-wip-us.apache.org/repos/asf/incubator-beam/blob/8017a025/runners/flink/README.md -- diff --git a/runners/flink/README.md b/runners/flink/README.md index 8361112..c0b3830 100644 --- a/runners/flink/README.md +++ b/runners/flink/README.md @@ -152,7 +152,7 @@ Maven project. mvn archetype:generate -DgroupId=com.mycompany.beam -DartifactId=beam-test \ -DarchetypeArtifactId=maven-archetype-quickstart -DinteractiveMode=false -The contents of the root `pom.xml` should be slightly changed aftewards (explanation below): +The contents of the root `pom.xml` should be slightly changed afterwards (explanation below): ```xml @@ -199,6 +199,14 @@ The contents of the root `pom.xml` should be slightly changed aftewards (explana org.apache.beam.runners.flink.examples.WordCount + + + *:* + +META-INF/LICENSE + + +
[2/2] incubator-beam git commit: This closes #1075
This closes #1075 Project: http://git-wip-us.apache.org/repos/asf/incubator-beam/repo Commit: http://git-wip-us.apache.org/repos/asf/incubator-beam/commit/6c882168 Tree: http://git-wip-us.apache.org/repos/asf/incubator-beam/tree/6c882168 Diff: http://git-wip-us.apache.org/repos/asf/incubator-beam/diff/6c882168 Branch: refs/heads/master Commit: 6c8821682f45f4361392c9da0756076ec5d0966f Parents: daf69f8 89cfee0 Author: Jean-Baptiste OnofréAuthored: Tue Oct 11 08:35:39 2016 +0200 Committer: Jean-Baptiste Onofré Committed: Tue Oct 11 08:35:39 2016 +0200 -- .../io/jdbc/src/main/java/org/apache/beam/sdk/io/jdbc/JdbcIO.java | 2 +- 1 file changed, 1 insertion(+), 1 deletion(-) --
[1/2] incubator-beam git commit: Define DataSourceConfiguration as public in JdbcIO
Repository: incubator-beam Updated Branches: refs/heads/master daf69f87e -> 6c8821682 Define DataSourceConfiguration as public in JdbcIO Project: http://git-wip-us.apache.org/repos/asf/incubator-beam/repo Commit: http://git-wip-us.apache.org/repos/asf/incubator-beam/commit/89cfee06 Tree: http://git-wip-us.apache.org/repos/asf/incubator-beam/tree/89cfee06 Diff: http://git-wip-us.apache.org/repos/asf/incubator-beam/diff/89cfee06 Branch: refs/heads/master Commit: 89cfee06f310c8137c3822d0096eadd359f5b8a2 Parents: daf69f8 Author: Jean-Baptiste OnofréAuthored: Mon Oct 10 21:15:04 2016 +0200 Committer: Jean-Baptiste Onofré Committed: Tue Oct 11 07:43:01 2016 +0200 -- .../io/jdbc/src/main/java/org/apache/beam/sdk/io/jdbc/JdbcIO.java | 2 +- 1 file changed, 1 insertion(+), 1 deletion(-) -- http://git-wip-us.apache.org/repos/asf/incubator-beam/blob/89cfee06/sdks/java/io/jdbc/src/main/java/org/apache/beam/sdk/io/jdbc/JdbcIO.java -- diff --git a/sdks/java/io/jdbc/src/main/java/org/apache/beam/sdk/io/jdbc/JdbcIO.java b/sdks/java/io/jdbc/src/main/java/org/apache/beam/sdk/io/jdbc/JdbcIO.java index f4c3cab..3bdbcce 100644 --- a/sdks/java/io/jdbc/src/main/java/org/apache/beam/sdk/io/jdbc/JdbcIO.java +++ b/sdks/java/io/jdbc/src/main/java/org/apache/beam/sdk/io/jdbc/JdbcIO.java @@ -136,7 +136,7 @@ public class JdbcIO { * properties allowing to create a {@link DataSource}. */ @AutoValue - abstract static class DataSourceConfiguration implements Serializable { + public abstract static class DataSourceConfiguration implements Serializable { @Nullable abstract String getDriverClassName(); @Nullable abstract String getUrl(); @Nullable abstract String getUsername();
[GitHub] incubator-beam pull request #1075: Define DataSourceConfiguration as public ...
GitHub user jbonofre opened a pull request: https://github.com/apache/incubator-beam/pull/1075 Define DataSourceConfiguration as public in JdbcIO Be sure to do all of the following to help us incorporate your contribution quickly and easily: - [ ] Make sure the PR title is formatted like: `[BEAM-] Description of pull request` - [X] Make sure tests pass via `mvn clean verify`. (Even better, enable Travis-CI on your fork and ensure the whole test matrix passes). - [ ] Replace `` in the title with the actual Jira issue number, if there is one. - [ ] If this contribution is large, please file an Apache [Individual Contributor License Agreement](https://www.apache.org/licenses/icla.txt). --- You can merge this pull request into a Git repository by running: $ git pull https://github.com/jbonofre/incubator-beam JDBCIO_DSCONFIG_PUBLIC Alternatively you can review and apply these changes as the patch at: https://github.com/apache/incubator-beam/pull/1075.patch To close this pull request, make a commit to your master/trunk branch with (at least) the following in the commit message: This closes #1075 commit 78c7d8b01a0ca55a17a8e66e2ce16b0f9f58e895 Author: Jean-Baptiste Onofré <jbono...@apache.org> Date: 2016-10-10T19:15:04Z Define DataSourceConfiguration as public in JdbcIO --- If your project is set up for it, you can reply to this email and have your reply appear on GitHub as well. If your project does not have this feature enabled and wishes so, or if the feature is enabled but not working, please contact infrastructure at infrastruct...@apache.org or file a JIRA ticket with INFRA. ---
[2/2] incubator-beam git commit: [BEAM-698] This closes #1054
[BEAM-698] This closes #1054 Project: http://git-wip-us.apache.org/repos/asf/incubator-beam/repo Commit: http://git-wip-us.apache.org/repos/asf/incubator-beam/commit/8130bc36 Tree: http://git-wip-us.apache.org/repos/asf/incubator-beam/tree/8130bc36 Diff: http://git-wip-us.apache.org/repos/asf/incubator-beam/diff/8130bc36 Branch: refs/heads/master Commit: 8130bc36feca77737a4e171e14307f53410201c7 Parents: f27354f 26474c7 Author: Jean-Baptiste OnofréAuthored: Thu Oct 6 14:02:34 2016 +0200 Committer: Jean-Baptiste Onofré Committed: Thu Oct 6 14:02:34 2016 +0200 -- .../beam/sdk/io/mongodb/MongoDbGridFSIO.java| 197 ++- .../apache/beam/sdk/io/mongodb/MongoDbIO.java | 20 +- .../sdk/io/mongodb/MongoDBGridFSIOTest.java | 13 +- 3 files changed, 117 insertions(+), 113 deletions(-) --
[GitHub] incubator-beam pull request #1054: [BEAM-698] Use AutoValue in MongoDB GridF...
GitHub user jbonofre opened a pull request: https://github.com/apache/incubator-beam/pull/1054 [BEAM-698] Use AutoValue in MongoDB GridFS Be sure to do all of the following to help us incorporate your contribution quickly and easily: - [X] Make sure the PR title is formatted like: `[BEAM-] Description of pull request` - [X] Make sure tests pass via `mvn clean verify`. (Even better, enable Travis-CI on your fork and ensure the whole test matrix passes). - [X] Replace `` in the title with the actual Jira issue number, if there is one. - [X] If this contribution is large, please file an Apache [Individual Contributor License Agreement](https://www.apache.org/licenses/icla.txt). --- You can merge this pull request into a Git repository by running: $ git pull https://github.com/jbonofre/incubator-beam BEAM-698-AUTOVALUE_GRIDFS Alternatively you can review and apply these changes as the patch at: https://github.com/apache/incubator-beam/pull/1054.patch To close this pull request, make a commit to your master/trunk branch with (at least) the following in the commit message: This closes #1054 commit 83d1ed7c06adf35a791c1fee96cd86f73424 Author: Jean-Baptiste Onofré <jbono...@apache.org> Date: 2016-10-03T06:02:07Z [BEAM-698] Use AutoValue in MongoDB GridFS --- If your project is set up for it, you can reply to this email and have your reply appear on GitHub as well. If your project does not have this feature enabled and wishes so, or if the feature is enabled but not working, please contact infrastructure at infrastruct...@apache.org or file a JIRA ticket with INFRA. ---
[2/2] incubator-beam git commit: [BEAM-698] This closes #1033
[BEAM-698] This closes #1033 Project: http://git-wip-us.apache.org/repos/asf/incubator-beam/repo Commit: http://git-wip-us.apache.org/repos/asf/incubator-beam/commit/202acd1d Tree: http://git-wip-us.apache.org/repos/asf/incubator-beam/tree/202acd1d Diff: http://git-wip-us.apache.org/repos/asf/incubator-beam/diff/202acd1d Branch: refs/heads/master Commit: 202acd1d6196b12b0d189b947ef202e32581cbed Parents: 2e0adaf 4d95423 Author: Jean-Baptiste OnofréAuthored: Sun Oct 2 15:51:40 2016 +0200 Committer: Jean-Baptiste Onofré Committed: Sun Oct 2 15:51:40 2016 +0200 -- sdks/java/io/mongodb/pom.xml| 6 + .../apache/beam/sdk/io/mongodb/MongoDbIO.java | 276 --- .../beam/sdk/io/mongodb/MongoDbIOTest.java | 19 +- 3 files changed, 132 insertions(+), 169 deletions(-) --
[1/2] incubator-beam git commit: [BEAM-698] Use AutoValue and deal with Document instead of String in MongoDbIO
Repository: incubator-beam Updated Branches: refs/heads/master 2e0adaf02 -> 202acd1d6 [BEAM-698] Use AutoValue and deal with Document instead of String in MongoDbIO Project: http://git-wip-us.apache.org/repos/asf/incubator-beam/repo Commit: http://git-wip-us.apache.org/repos/asf/incubator-beam/commit/4d95423b Tree: http://git-wip-us.apache.org/repos/asf/incubator-beam/tree/4d95423b Diff: http://git-wip-us.apache.org/repos/asf/incubator-beam/diff/4d95423b Branch: refs/heads/master Commit: 4d95423bfe5ede1c48fea9489a18c17079691088 Parents: 2e0adaf Author: Eugene KirpichovAuthored: Wed Sep 28 17:17:22 2016 -0700 Committer: Jean-Baptiste Onofré Committed: Sun Oct 2 15:49:49 2016 +0200 -- sdks/java/io/mongodb/pom.xml| 6 + .../apache/beam/sdk/io/mongodb/MongoDbIO.java | 276 --- .../beam/sdk/io/mongodb/MongoDbIOTest.java | 19 +- 3 files changed, 132 insertions(+), 169 deletions(-) -- http://git-wip-us.apache.org/repos/asf/incubator-beam/blob/4d95423b/sdks/java/io/mongodb/pom.xml -- diff --git a/sdks/java/io/mongodb/pom.xml b/sdks/java/io/mongodb/pom.xml index b7e36af..173 100644 --- a/sdks/java/io/mongodb/pom.xml +++ b/sdks/java/io/mongodb/pom.xml @@ -94,6 +94,12 @@ joda-time + + com.google.auto.value + auto-value + provided + + de.flapdoodle.embed http://git-wip-us.apache.org/repos/asf/incubator-beam/blob/4d95423b/sdks/java/io/mongodb/src/main/java/org/apache/beam/sdk/io/mongodb/MongoDbIO.java -- diff --git a/sdks/java/io/mongodb/src/main/java/org/apache/beam/sdk/io/mongodb/MongoDbIO.java b/sdks/java/io/mongodb/src/main/java/org/apache/beam/sdk/io/mongodb/MongoDbIO.java index a54694a..7c2bc28 100644 --- a/sdks/java/io/mongodb/src/main/java/org/apache/beam/sdk/io/mongodb/MongoDbIO.java +++ b/sdks/java/io/mongodb/src/main/java/org/apache/beam/sdk/io/mongodb/MongoDbIO.java @@ -17,9 +17,10 @@ */ package org.apache.beam.sdk.io.mongodb; +import static com.google.common.base.Preconditions.checkArgument; import static com.google.common.base.Preconditions.checkNotNull; -import com.google.common.annotations.VisibleForTesting; +import com.google.auto.value.AutoValue; import com.mongodb.BasicDBObject; import com.mongodb.MongoClient; import com.mongodb.MongoClientURI; @@ -98,12 +99,12 @@ public class MongoDbIO { /** Read data from MongoDB. */ public static Read read() { -return new Read(new BoundedMongoDbSource(null, null, null, null, 0)); +return new AutoValue_MongoDbIO_Read.Builder().setNumSplits(0).build(); } /** Write data to MongoDB. */ public static Write write() { -return new Write(new Write.MongoDbWriter(null, null, null, 1024L)); +return new AutoValue_MongoDbIO_Write.Builder().setBatchSize(1024L).build(); } private MongoDbIO() { @@ -112,118 +113,97 @@ public class MongoDbIO { /** * A {@link PTransform} to read data from MongoDB. */ - public static class Read extends PTransform { + @AutoValue + public abstract static class Read extends PTransform { +@Nullable abstract String uri(); +@Nullable abstract String database(); +@Nullable abstract String collection(); +@Nullable abstract String filter(); +abstract int numSplits(); + +abstract Builder toBuilder(); + +@AutoValue.Builder +abstract static class Builder { + abstract Builder setUri(String uri); + abstract Builder setDatabase(String database); + abstract Builder setCollection(String collection); + abstract Builder setFilter(String filter); + abstract Builder setNumSplits(int numSplits); + abstract Read build(); +} +/** + * Example documentation for withUri. + */ public Read withUri(String uri) { - return new Read(source.withUri(uri)); + checkNotNull(uri); + return toBuilder().setUri(uri).build(); } public Read withDatabase(String database) { - return new Read(source.withDatabase(database)); + checkNotNull(database); + return toBuilder().setDatabase(database).build(); } public Read withCollection(String collection) { - return new Read(source.withCollection(collection)); + checkNotNull(collection); + return toBuilder().setCollection(collection).build(); } public Read withFilter(String filter) { - return new Read(source.withFilter(filter)); + checkNotNull(filter); + return toBuilder().setFilter(filter).build(); } public Read withNumSplits(int numSplits) { - return new Read(source.withNumSplits(numSplits)); -} - -private
[2/2] incubator-beam git commit: [BEAM-674] This closes #1025
[BEAM-674] This closes #1025 Project: http://git-wip-us.apache.org/repos/asf/incubator-beam/repo Commit: http://git-wip-us.apache.org/repos/asf/incubator-beam/commit/2e0adaf0 Tree: http://git-wip-us.apache.org/repos/asf/incubator-beam/tree/2e0adaf0 Diff: http://git-wip-us.apache.org/repos/asf/incubator-beam/diff/2e0adaf0 Branch: refs/heads/master Commit: 2e0adaf0223adde897cd1b2134014db673474fe8 Parents: c5c3436 54854f8 Author: Jean-Baptiste OnofréAuthored: Sun Oct 2 15:12:42 2016 +0200 Committer: Jean-Baptiste Onofré Committed: Sun Oct 2 15:12:42 2016 +0200 -- .../beam/sdk/io/mongodb/MongoDbGridFSIO.java| 403 ++- .../sdk/io/mongodb/MongoDBGridFSIOTest.java | 100 +++-- 2 files changed, 270 insertions(+), 233 deletions(-) --
[1/2] incubator-beam git commit: [BEAM-674] Refactoring and improvements on the MongoDB GridFS IO
Repository: incubator-beam Updated Branches: refs/heads/master c5c343659 -> 2e0adaf02 [BEAM-674] Refactoring and improvements on the MongoDB GridFS IO Project: http://git-wip-us.apache.org/repos/asf/incubator-beam/repo Commit: http://git-wip-us.apache.org/repos/asf/incubator-beam/commit/54854f86 Tree: http://git-wip-us.apache.org/repos/asf/incubator-beam/tree/54854f86 Diff: http://git-wip-us.apache.org/repos/asf/incubator-beam/diff/54854f86 Branch: refs/heads/master Commit: 54854f86346f805008c0d459caf402dd0ad4e46d Parents: c5c3436 Author: Daniel KulpAuthored: Wed Sep 28 22:44:37 2016 -0400 Committer: Jean-Baptiste Onofré Committed: Sun Oct 2 15:11:39 2016 +0200 -- .../beam/sdk/io/mongodb/MongoDbGridFSIO.java| 403 ++- .../sdk/io/mongodb/MongoDBGridFSIOTest.java | 100 +++-- 2 files changed, 270 insertions(+), 233 deletions(-) -- http://git-wip-us.apache.org/repos/asf/incubator-beam/blob/54854f86/sdks/java/io/mongodb/src/main/java/org/apache/beam/sdk/io/mongodb/MongoDbGridFSIO.java -- diff --git a/sdks/java/io/mongodb/src/main/java/org/apache/beam/sdk/io/mongodb/MongoDbGridFSIO.java b/sdks/java/io/mongodb/src/main/java/org/apache/beam/sdk/io/mongodb/MongoDbGridFSIO.java index 337e5f5..cebda64 100644 --- a/sdks/java/io/mongodb/src/main/java/org/apache/beam/sdk/io/mongodb/MongoDbGridFSIO.java +++ b/sdks/java/io/mongodb/src/main/java/org/apache/beam/sdk/io/mongodb/MongoDbGridFSIO.java @@ -17,6 +17,7 @@ */ package org.apache.beam.sdk.io.mongodb; +import static com.google.common.base.Preconditions.checkNotNull; import com.mongodb.DB; import com.mongodb.DBCursor; import com.mongodb.DBObject; @@ -30,8 +31,8 @@ import java.io.BufferedReader; import java.io.IOException; import java.io.InputStreamReader; import java.io.Serializable; +import java.util.ArrayList; import java.util.Iterator; -import java.util.LinkedList; import java.util.List; import java.util.NoSuchElementException; @@ -42,11 +43,14 @@ import org.apache.beam.sdk.coders.SerializableCoder; import org.apache.beam.sdk.coders.StringUtf8Coder; import org.apache.beam.sdk.io.BoundedSource; import org.apache.beam.sdk.options.PipelineOptions; +import org.apache.beam.sdk.transforms.DoFn; import org.apache.beam.sdk.transforms.PTransform; +import org.apache.beam.sdk.transforms.ParDo; import org.apache.beam.sdk.transforms.display.DisplayData; import org.apache.beam.sdk.values.PBegin; import org.apache.beam.sdk.values.PCollection; import org.bson.types.ObjectId; +import org.joda.time.Duration; import org.joda.time.Instant; @@ -55,10 +59,12 @@ import org.joda.time.Instant; * * Reading from MongoDB via GridFS * - * MongoDbGridFSIO source returns a bounded collection of String as {@code PCollection}. + * MongoDbGridFSIO source returns a bounded collection of Objects as {@code PCollection}. * - * To configure the MongoDB source, you have to provide the connection URI, the database name - * and the bucket name. The following example illustrates various options for configuring the + * To configure the MongoDB GridFS source, you can provide the connection URI, the database name + * and the bucket name. If unspecified, the default values from the GridFS driver are used. + * + * The following example illustrates various options for configuring the * source: * * {@code @@ -73,132 +79,172 @@ import org.joda.time.Instant; * The source also accepts an optional configuration: {@code withQueryFilter()} allows you to * define a JSON filter to get subset of files in the database. * - * There is also an optional {@code ParseCallback} that can be specified that can be used to + * There is also an optional {@code Parser} that can be specified that can be used to * parse the InputStream into objects usable with Beam. By default, MongoDbGridFSIO will parse * into Strings, splitting on line breaks and using the uploadDate of the file as the timestamp. + * When using a parser that outputs with custom timestamps, you may also need to specify + * the allowedTimestampSkew option. */ public class MongoDbGridFSIO { /** - * Function for parsing the GridFSDBFile into objects for the PCollection. - * @param + * Callback for the parser to use to submit data. */ - public interface ParseCallback extends Serializable { + public interface ParserCallback extends Serializable { /** - * Each value parsed from the file should be output as an - * Iterable of LineT. If timestamp is omitted, it will - * use the uploadDate of the GridFSDBFile. + * Output the object. The default timestamp will be the GridFSDBFile + * creation timestamp. + * @param output */ -public static class Line { - final
[1/2] incubator-beam git commit: [BEAM-244] Add JDBC IO
Repository: incubator-beam Updated Branches: refs/heads/master bc80ee342 -> c5c343659 [BEAM-244] Add JDBC IO Project: http://git-wip-us.apache.org/repos/asf/incubator-beam/repo Commit: http://git-wip-us.apache.org/repos/asf/incubator-beam/commit/19fad184 Tree: http://git-wip-us.apache.org/repos/asf/incubator-beam/tree/19fad184 Diff: http://git-wip-us.apache.org/repos/asf/incubator-beam/diff/19fad184 Branch: refs/heads/master Commit: 19fad184ac0f8521770dff96bdad5bff2ef9aa03 Parents: bc80ee3 Author: Jean-Baptiste OnofréAuthored: Mon Sep 5 12:57:14 2016 +0200 Committer: Jean-Baptiste Onofré Committed: Sun Oct 2 10:35:03 2016 +0200 -- sdks/java/io/jdbc/pom.xml | 138 ++ .../org/apache/beam/sdk/io/jdbc/JdbcIO.java | 427 +++ .../apache/beam/sdk/io/jdbc/package-info.java | 22 + .../org/apache/beam/sdk/io/jdbc/JdbcIOTest.java | 236 ++ sdks/java/io/pom.xml| 1 + 5 files changed, 824 insertions(+) -- http://git-wip-us.apache.org/repos/asf/incubator-beam/blob/19fad184/sdks/java/io/jdbc/pom.xml -- diff --git a/sdks/java/io/jdbc/pom.xml b/sdks/java/io/jdbc/pom.xml new file mode 100644 index 000..75eb5ed --- /dev/null +++ b/sdks/java/io/jdbc/pom.xml @@ -0,0 +1,138 @@ + + +http://maven.apache.org/POM/4.0.0; xmlns:xsi="http://www.w3.org/2001/XMLSchema-instance; xsi:schemaLocation="http://maven.apache.org/POM/4.0.0 http://maven.apache.org/xsd/maven-4.0.0.xsd;> + + 4.0.0 + + +org.apache.beam +beam-sdks-java-io-parent +0.3.0-incubating-SNAPSHOT +../pom.xml + + + beam-sdks-java-io-jdbc + Apache Beam :: SDKs :: Java :: IO :: JDBC + IO to read and write on JDBC datasource. + + + + +org.apache.maven.plugins +maven-compiler-plugin + + +org.apache.maven.plugins +maven-source-plugin + + +org.apache.maven.plugins +maven-surefire-plugin + + +org.apache.maven.plugins +maven-jar-plugin + + +org.apache.maven.plugins +maven-checkstyle-plugin + + +org.apache.maven.plugins +maven-javadoc-plugin + + + + + + + org.apache.beam + beam-sdks-java-core + + + + org.slf4j + slf4j-api + + + + com.google.guava + guava + + + + com.google.code.findbugs + annotations + + + + org.apache.commons + commons-dbcp2 + 2.1.1 + + + + + com.google.auto.value + auto-value + provided + + + + + org.apache.derby + derby + 10.12.1.1 + test + + + org.apache.derby + derbyclient + 10.12.1.1 + test + + + org.apache.derby + derbynet + 10.12.1.1 + test + + + org.apache.beam + beam-runners-direct-java + ${project.version} + test + + + junit + junit + test + + + org.hamcrest + hamcrest-all + test + + + org.slf4j + slf4j-jdk14 + test + + + + \ No newline at end of file http://git-wip-us.apache.org/repos/asf/incubator-beam/blob/19fad184/sdks/java/io/jdbc/src/main/java/org/apache/beam/sdk/io/jdbc/JdbcIO.java -- diff --git a/sdks/java/io/jdbc/src/main/java/org/apache/beam/sdk/io/jdbc/JdbcIO.java b/sdks/java/io/jdbc/src/main/java/org/apache/beam/sdk/io/jdbc/JdbcIO.java new file mode 100644 index 000..f4c3cab --- /dev/null +++ b/sdks/java/io/jdbc/src/main/java/org/apache/beam/sdk/io/jdbc/JdbcIO.java @@ -0,0 +1,427 @@ +/* + * 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. + */ +package org.apache.beam.sdk.io.jdbc; + +import static com.google.common.base.Preconditions.checkArgument; +import static com.google.common.base.Preconditions.checkNotNull; + +import com.google.auto.value.AutoValue; + +import
[2/2] incubator-beam git commit: [BEAM-244] This closes #942
[BEAM-244] This closes #942 Project: http://git-wip-us.apache.org/repos/asf/incubator-beam/repo Commit: http://git-wip-us.apache.org/repos/asf/incubator-beam/commit/c5c34365 Tree: http://git-wip-us.apache.org/repos/asf/incubator-beam/tree/c5c34365 Diff: http://git-wip-us.apache.org/repos/asf/incubator-beam/diff/c5c34365 Branch: refs/heads/master Commit: c5c343659ea7a597b2b6a5fe7efcec001f17a8f9 Parents: bc80ee3 19fad18 Author: Jean-Baptiste OnofréAuthored: Sun Oct 2 10:37:27 2016 +0200 Committer: Jean-Baptiste Onofré Committed: Sun Oct 2 10:37:27 2016 +0200 -- sdks/java/io/jdbc/pom.xml | 138 ++ .../org/apache/beam/sdk/io/jdbc/JdbcIO.java | 427 +++ .../apache/beam/sdk/io/jdbc/package-info.java | 22 + .../org/apache/beam/sdk/io/jdbc/JdbcIOTest.java | 236 ++ sdks/java/io/pom.xml| 1 + 5 files changed, 824 insertions(+) --
[1/2] incubator-beam git commit: [BEAM-674] Add GridFS support to MongoDbIO
Repository: incubator-beam Updated Branches: refs/heads/master 3879db036 -> 307d592d2 [BEAM-674] Add GridFS support to MongoDbIO Project: http://git-wip-us.apache.org/repos/asf/incubator-beam/repo Commit: http://git-wip-us.apache.org/repos/asf/incubator-beam/commit/68c8c787 Tree: http://git-wip-us.apache.org/repos/asf/incubator-beam/tree/68c8c787 Diff: http://git-wip-us.apache.org/repos/asf/incubator-beam/diff/68c8c787 Branch: refs/heads/master Commit: 68c8c7872720f4e8fbcd017032c0e90e395e905c Parents: 3879db0 Author: Daniel KulpAuthored: Fri Sep 16 16:58:56 2016 -0400 Committer: Jean-Baptiste Onofré Committed: Wed Sep 28 17:18:46 2016 +0200 -- sdks/java/io/mongodb/pom.xml| 6 +- .../beam/sdk/io/mongodb/MongoDbGridFSIO.java| 427 +++ .../sdk/io/mongodb/MongoDBGridFSIOTest.java | 257 +++ 3 files changed, 689 insertions(+), 1 deletion(-) -- http://git-wip-us.apache.org/repos/asf/incubator-beam/blob/68c8c787/sdks/java/io/mongodb/pom.xml -- diff --git a/sdks/java/io/mongodb/pom.xml b/sdks/java/io/mongodb/pom.xml index 60f1d1e..b7e36af 100644 --- a/sdks/java/io/mongodb/pom.xml +++ b/sdks/java/io/mongodb/pom.xml @@ -89,6 +89,10 @@ mongo-java-driver ${mongo-java-driver.version} + + joda-time + joda-time + @@ -126,4 +130,4 @@ - \ No newline at end of file + http://git-wip-us.apache.org/repos/asf/incubator-beam/blob/68c8c787/sdks/java/io/mongodb/src/main/java/org/apache/beam/sdk/io/mongodb/MongoDbGridFSIO.java -- diff --git a/sdks/java/io/mongodb/src/main/java/org/apache/beam/sdk/io/mongodb/MongoDbGridFSIO.java b/sdks/java/io/mongodb/src/main/java/org/apache/beam/sdk/io/mongodb/MongoDbGridFSIO.java new file mode 100644 index 000..337e5f5 --- /dev/null +++ b/sdks/java/io/mongodb/src/main/java/org/apache/beam/sdk/io/mongodb/MongoDbGridFSIO.java @@ -0,0 +1,427 @@ +/* + * 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. + */ +package org.apache.beam.sdk.io.mongodb; + +import com.mongodb.DB; +import com.mongodb.DBCursor; +import com.mongodb.DBObject; +import com.mongodb.Mongo; +import com.mongodb.MongoURI; +import com.mongodb.gridfs.GridFS; +import com.mongodb.gridfs.GridFSDBFile; +import com.mongodb.util.JSON; + +import java.io.BufferedReader; +import java.io.IOException; +import java.io.InputStreamReader; +import java.io.Serializable; +import java.util.Iterator; +import java.util.LinkedList; +import java.util.List; +import java.util.NoSuchElementException; + +import javax.annotation.Nullable; + +import org.apache.beam.sdk.coders.Coder; +import org.apache.beam.sdk.coders.SerializableCoder; +import org.apache.beam.sdk.coders.StringUtf8Coder; +import org.apache.beam.sdk.io.BoundedSource; +import org.apache.beam.sdk.options.PipelineOptions; +import org.apache.beam.sdk.transforms.PTransform; +import org.apache.beam.sdk.transforms.display.DisplayData; +import org.apache.beam.sdk.values.PBegin; +import org.apache.beam.sdk.values.PCollection; +import org.bson.types.ObjectId; +import org.joda.time.Instant; + + +/** + * IO to read and write data on MongoDB GridFS. + * + * Reading from MongoDB via GridFS + * + * MongoDbGridFSIO source returns a bounded collection of String as {@code PCollection}. + * + * To configure the MongoDB source, you have to provide the connection URI, the database name + * and the bucket name. The following example illustrates various options for configuring the + * source: + * + * {@code + * + * pipeline.apply(MongoDbGridFSIO.read() + * .withUri("mongodb://localhost:27017") + * .withDatabase("my-database") + * .withBucket("my-bucket")) + * + * } + * + * The source also accepts an optional configuration: {@code withQueryFilter()} allows you to + * define a JSON filter to get subset of files in the database. + * + * There is also an optional {@code ParseCallback} that can be specified that can be
[2/2] incubator-beam git commit: [BEAM-674] This closes #1003
[BEAM-674] This closes #1003 Project: http://git-wip-us.apache.org/repos/asf/incubator-beam/repo Commit: http://git-wip-us.apache.org/repos/asf/incubator-beam/commit/307d592d Tree: http://git-wip-us.apache.org/repos/asf/incubator-beam/tree/307d592d Diff: http://git-wip-us.apache.org/repos/asf/incubator-beam/diff/307d592d Branch: refs/heads/master Commit: 307d592d2fd3a6d7bd78fc6243292ff8045b3fdc Parents: 3879db0 68c8c78 Author: Jean-Baptiste OnofréAuthored: Wed Sep 28 17:54:52 2016 +0200 Committer: Jean-Baptiste Onofré Committed: Wed Sep 28 17:54:52 2016 +0200 -- sdks/java/io/mongodb/pom.xml| 6 +- .../beam/sdk/io/mongodb/MongoDbGridFSIO.java| 427 +++ .../sdk/io/mongodb/MongoDBGridFSIOTest.java | 257 +++ 3 files changed, 689 insertions(+), 1 deletion(-) --
[GitHub] incubator-beam pull request #971: [BEAM-606] Create MqttIO
GitHub user jbonofre opened a pull request: https://github.com/apache/incubator-beam/pull/971 [BEAM-606] Create MqttIO Be sure to do all of the following to help us incorporate your contribution quickly and easily: - [X] Make sure the PR title is formatted like: `[BEAM-] Description of pull request` - [X] Make sure tests pass via `mvn clean verify`. (Even better, enable Travis-CI on your fork and ensure the whole test matrix passes). - [X] Replace `` in the title with the actual Jira issue number, if there is one. - [X] If this contribution is large, please file an Apache [Individual Contributor License Agreement](https://www.apache.org/licenses/icla.txt). --- You can merge this pull request into a Git repository by running: $ git pull https://github.com/jbonofre/incubator-beam BEAM-606-MQTTIO Alternatively you can review and apply these changes as the patch at: https://github.com/apache/incubator-beam/pull/971.patch To close this pull request, make a commit to your master/trunk branch with (at least) the following in the commit message: This closes #971 commit f826b4a8a671dc6a02067d9e5feda6a4c1ab3405 Author: Jean-Baptiste Onofré <jbono...@apache.org> Date: 2016-09-12T16:49:36Z [BEAM-606] Create MqttIO --- If your project is set up for it, you can reply to this email and have your reply appear on GitHub as well. If your project does not have this feature enabled and wishes so, or if the feature is enabled but not working, please contact infrastructure at infrastruct...@apache.org or file a JIRA ticket with INFRA. ---
[2/2] incubator-beam git commit: [BEAM-634] This closes #949
[BEAM-634] This closes #949 Project: http://git-wip-us.apache.org/repos/asf/incubator-beam/repo Commit: http://git-wip-us.apache.org/repos/asf/incubator-beam/commit/c4036753 Tree: http://git-wip-us.apache.org/repos/asf/incubator-beam/tree/c4036753 Diff: http://git-wip-us.apache.org/repos/asf/incubator-beam/diff/c4036753 Branch: refs/heads/master Commit: c4036753fe95708cfd14bd360c60bdfd7a4ec953 Parents: d71d828 a0ae04b Author: Jean-Baptiste OnofréAuthored: Thu Sep 15 16:59:07 2016 +0200 Committer: Jean-Baptiste Onofré Committed: Thu Sep 15 16:59:07 2016 +0200 -- pom.xml | 35 ++-- runners/direct-java/pom.xml | 8 - .../direct/BoundedReadEvaluatorFactory.java | 9 ++--- .../direct/ParDoMultiEvaluatorFactory.java | 12 +++ .../direct/ParDoSingleEvaluatorFactory.java | 9 ++--- .../direct/UnboundedReadEvaluatorFactory.java | 9 ++--- .../apache/beam/runners/flink/package-info.java | 22 .../src/main/resources/beam/checkstyle.xml | 3 ++ .../src/main/resources/beam/suppressions.xml| 5 +++ .../beam/sdk/coders/IterableLikeCoder.java | 22 ++-- .../beam/sdk/coders/protobuf/ProtoCoder.java| 4 ++- .../beam/sdk/util/MergingActiveWindowSet.java | 12 --- .../org/apache/beam/sdk/util/PubsubClient.java | 2 +- .../beam/sdk/util/PubsubJsonClientTest.java | 16 - .../apache/beam/sdk/io/kinesis/KinesisIO.java | 4 ++- .../beam/sdk/io/kinesis/KinesisUploader.java| 3 +- .../beam/sdk/io/kinesis/package-info.java | 22 .../beam/sdk/io/mongodb/package-info.java | 22 18 files changed, 102 insertions(+), 117 deletions(-) --
[GitHub] incubator-beam pull request #942: [BEAM-244] Add JDBC IO
GitHub user jbonofre opened a pull request: https://github.com/apache/incubator-beam/pull/942 [BEAM-244] Add JDBC IO Be sure to do all of the following to help us incorporate your contribution quickly and easily: - [X] Make sure the PR title is formatted like: `[BEAM-] Description of pull request` - [X] Make sure tests pass via `mvn clean verify`. (Even better, enable Travis-CI on your fork and ensure the whole test matrix passes). - [X] Replace `` in the title with the actual Jira issue number, if there is one. - [X] If this contribution is large, please file an Apache [Individual Contributor License Agreement](https://www.apache.org/licenses/icla.txt). --- You can merge this pull request into a Git repository by running: $ git pull https://github.com/jbonofre/incubator-beam BEAM-244-JDBCIO Alternatively you can review and apply these changes as the patch at: https://github.com/apache/incubator-beam/pull/942.patch To close this pull request, make a commit to your master/trunk branch with (at least) the following in the commit message: This closes #942 commit 6d77b13d1d1b61beedb0c8ef7b761d2bba9e59f5 Author: Jean-Baptiste Onofré <jbono...@apache.org> Date: 2016-09-05T10:57:14Z [BEAM-244] Add JDBC IO --- If your project is set up for it, you can reply to this email and have your reply appear on GitHub as well. If your project does not have this feature enabled and wishes so, or if the feature is enabled but not working, please contact infrastructure at infrastruct...@apache.org or file a JIRA ticket with INFRA. ---
[1/2] incubator-beam git commit: [BEAM-242] Enable and fix checkstyle in Flink runner examples
Repository: incubator-beam Updated Branches: refs/heads/master 387854624 -> 26635d7fb [BEAM-242] Enable and fix checkstyle in Flink runner examples Project: http://git-wip-us.apache.org/repos/asf/incubator-beam/repo Commit: http://git-wip-us.apache.org/repos/asf/incubator-beam/commit/dafb8055 Tree: http://git-wip-us.apache.org/repos/asf/incubator-beam/tree/dafb8055 Diff: http://git-wip-us.apache.org/repos/asf/incubator-beam/diff/dafb8055 Branch: refs/heads/master Commit: dafb80556c1d984630c6ccf615ba982903f176df Parents: 3878546 Author: Jean-Baptiste OnofréAuthored: Tue Sep 6 07:26:45 2016 +0200 Committer: Jean-Baptiste Onofré Committed: Wed Sep 7 05:55:50 2016 +0200 -- runners/flink/examples/pom.xml | 2 -- .../beam/runners/flink/examples/WordCount.java | 9 ++ .../runners/flink/examples/package-info.java| 22 + .../flink/examples/streaming/AutoComplete.java | 5 +-- .../flink/examples/streaming/JoinExamples.java | 3 +- .../examples/streaming/KafkaIOExamples.java | 34 ++-- .../KafkaWindowedWordCountExample.java | 27 +--- .../examples/streaming/WindowedWordCount.java | 19 +++ .../flink/examples/streaming/package-info.java | 22 + 9 files changed, 110 insertions(+), 33 deletions(-) -- http://git-wip-us.apache.org/repos/asf/incubator-beam/blob/dafb8055/runners/flink/examples/pom.xml -- diff --git a/runners/flink/examples/pom.xml b/runners/flink/examples/pom.xml index 9f705db..b8489fc 100644 --- a/runners/flink/examples/pom.xml +++ b/runners/flink/examples/pom.xml @@ -109,12 +109,10 @@ - org.apache.maven.plugins http://git-wip-us.apache.org/repos/asf/incubator-beam/blob/dafb8055/runners/flink/examples/src/main/java/org/apache/beam/runners/flink/examples/WordCount.java -- diff --git a/runners/flink/examples/src/main/java/org/apache/beam/runners/flink/examples/WordCount.java b/runners/flink/examples/src/main/java/org/apache/beam/runners/flink/examples/WordCount.java index ab9297f..9cce757 100644 --- a/runners/flink/examples/src/main/java/org/apache/beam/runners/flink/examples/WordCount.java +++ b/runners/flink/examples/src/main/java/org/apache/beam/runners/flink/examples/WordCount.java @@ -36,8 +36,14 @@ import org.apache.beam.sdk.transforms.Sum; import org.apache.beam.sdk.values.KV; import org.apache.beam.sdk.values.PCollection; +/** + * Wordcount pipeline. + */ public class WordCount { + /** + * Function to extract words. + */ public static class ExtractWordsFn extends DoFn { private final Aggregator emptyLines = createAggregator("emptyLines", new Sum.SumLongFn()); @@ -60,6 +66,9 @@ public class WordCount { } } + /** + * PTransform counting words. + */ public static class CountWords extends PTransform >> { @Override http://git-wip-us.apache.org/repos/asf/incubator-beam/blob/dafb8055/runners/flink/examples/src/main/java/org/apache/beam/runners/flink/examples/package-info.java -- diff --git a/runners/flink/examples/src/main/java/org/apache/beam/runners/flink/examples/package-info.java b/runners/flink/examples/src/main/java/org/apache/beam/runners/flink/examples/package-info.java new file mode 100644 index 000..b0ecb56 --- /dev/null +++ b/runners/flink/examples/src/main/java/org/apache/beam/runners/flink/examples/package-info.java @@ -0,0 +1,22 @@ +/* + * 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. + */ + +/** + * Flink Beam runner exemple. + */ +package org.apache.beam.runners.flink.examples; http://git-wip-us.apache.org/repos/asf/incubator-beam/blob/dafb8055/runners/flink/examples/src/main/java/org/apache/beam/runners/flink/examples/streaming/AutoComplete.java
[2/2] incubator-beam git commit: [BEAM-242] This closes #919
[BEAM-242] This closes #919 Project: http://git-wip-us.apache.org/repos/asf/incubator-beam/repo Commit: http://git-wip-us.apache.org/repos/asf/incubator-beam/commit/26635d7f Tree: http://git-wip-us.apache.org/repos/asf/incubator-beam/tree/26635d7f Diff: http://git-wip-us.apache.org/repos/asf/incubator-beam/diff/26635d7f Branch: refs/heads/master Commit: 26635d7fb3d92185845d269909a3d399099df7da Parents: 3878546 dafb805 Author: Jean-Baptiste OnofréAuthored: Wed Sep 7 06:11:02 2016 +0200 Committer: Jean-Baptiste Onofré Committed: Wed Sep 7 06:11:02 2016 +0200 -- runners/flink/examples/pom.xml | 2 -- .../beam/runners/flink/examples/WordCount.java | 9 ++ .../runners/flink/examples/package-info.java| 22 + .../flink/examples/streaming/AutoComplete.java | 5 +-- .../flink/examples/streaming/JoinExamples.java | 3 +- .../examples/streaming/KafkaIOExamples.java | 34 ++-- .../KafkaWindowedWordCountExample.java | 27 +--- .../examples/streaming/WindowedWordCount.java | 19 +++ .../flink/examples/streaming/package-info.java | 22 + 9 files changed, 110 insertions(+), 33 deletions(-) --
[1/2] incubator-beam git commit: [BEAM-242] Enable checkstyle and fix checkstyle errors in Flink runner
Repository: incubator-beam Updated Branches: refs/heads/master 8ca683026 -> be689df07 [BEAM-242] Enable checkstyle and fix checkstyle errors in Flink runner Project: http://git-wip-us.apache.org/repos/asf/incubator-beam/repo Commit: http://git-wip-us.apache.org/repos/asf/incubator-beam/commit/95145574 Tree: http://git-wip-us.apache.org/repos/asf/incubator-beam/tree/95145574 Diff: http://git-wip-us.apache.org/repos/asf/incubator-beam/diff/95145574 Branch: refs/heads/master Commit: 951455746cebe6f42d2e7e85c02fd7c7be16767e Parents: 8ca6830 Author: Jean-Baptiste OnofréAuthored: Thu Aug 25 16:19:54 2016 +0200 Committer: Jean-Baptiste Onofré Committed: Mon Sep 5 15:27:08 2016 +0200 -- runners/flink/runner/pom.xml| 2 - .../FlinkPipelineExecutionEnvironment.java | 17 +- .../runners/flink/FlinkPipelineOptions.java | 27 ++-- .../runners/flink/FlinkRunnerRegistrar.java | 6 + .../beam/runners/flink/FlinkRunnerResult.java | 17 +- .../beam/runners/flink/TestFlinkRunner.java | 8 +- .../apache/beam/runners/flink/package-info.java | 22 +++ .../FlinkBatchPipelineTranslator.java | 15 +- .../FlinkBatchTranslationContext.java | 10 +- .../translation/FlinkPipelineTranslator.java| 2 +- .../FlinkStreamingTransformTranslators.java | 5 +- .../flink/translation/TranslationMode.java | 8 +- .../translation/functions/package-info.java | 22 +++ .../runners/flink/translation/package-info.java | 22 +++ .../translation/types/CoderTypeSerializer.java | 2 +- .../types/EncodedValueSerializer.java | 162 ++- .../flink/translation/types/package-info.java | 22 +++ .../utils/SerializedPipelineOptions.java| 2 +- .../flink/translation/utils/package-info.java | 22 +++ .../wrappers/DataOutputViewWrapper.java | 2 +- .../translation/wrappers/package-info.java | 22 +++ .../wrappers/streaming/DoFnOperator.java| 12 +- .../streaming/SingletonKeyedWorkItem.java | 5 + .../streaming/SingletonKeyedWorkItemCoder.java | 14 +- .../wrappers/streaming/WindowDoFnOperator.java | 2 +- .../wrappers/streaming/WorkItemKeySelector.java | 3 +- .../streaming/io/UnboundedFlinkSink.java| 13 +- .../streaming/io/UnboundedFlinkSource.java | 29 ++-- .../streaming/io/UnboundedSocketSource.java | 46 -- .../wrappers/streaming/io/package-info.java | 22 +++ .../wrappers/streaming/package-info.java| 22 +++ .../beam/runners/flink/PipelineOptionsTest.java | 3 + .../beam/runners/flink/WriteSinkITCase.java | 3 +- .../apache/beam/runners/flink/package-info.java | 22 +++ .../streaming/FlinkStateInternalsTest.java | 3 +- .../flink/streaming/GroupByNullKeyTest.java | 6 + .../streaming/TopWikipediaSessionsITCase.java | 2 +- .../streaming/UnboundedSourceWrapperTest.java | 33 ++-- .../runners/flink/streaming/package-info.java | 22 +++ 39 files changed, 490 insertions(+), 189 deletions(-) -- http://git-wip-us.apache.org/repos/asf/incubator-beam/blob/95145574/runners/flink/runner/pom.xml -- diff --git a/runners/flink/runner/pom.xml b/runners/flink/runner/pom.xml index 08adc60..7c32280 100644 --- a/runners/flink/runner/pom.xml +++ b/runners/flink/runner/pom.xml @@ -234,12 +234,10 @@ - org.apache.maven.plugins http://git-wip-us.apache.org/repos/asf/incubator-beam/blob/95145574/runners/flink/runner/src/main/java/org/apache/beam/runners/flink/FlinkPipelineExecutionEnvironment.java -- diff --git a/runners/flink/runner/src/main/java/org/apache/beam/runners/flink/FlinkPipelineExecutionEnvironment.java b/runners/flink/runner/src/main/java/org/apache/beam/runners/flink/FlinkPipelineExecutionEnvironment.java index d1977a4..a5d33b4 100644 --- a/runners/flink/runner/src/main/java/org/apache/beam/runners/flink/FlinkPipelineExecutionEnvironment.java +++ b/runners/flink/runner/src/main/java/org/apache/beam/runners/flink/FlinkPipelineExecutionEnvironment.java @@ -37,14 +37,15 @@ import org.slf4j.LoggerFactory; /** * The class that instantiates and manages the execution of a given job. * Depending on if the job is a Streaming or Batch processing one, it creates - * the adequate execution environment ({@link ExecutionEnvironment} or {@link StreamExecutionEnvironment}), - * the necessary {@link FlinkPipelineTranslator} ({@link FlinkBatchPipelineTranslator} or - * {@link FlinkStreamingPipelineTranslator}) to transform the Beam job into a Flink one, and - * executes the (translated) job. + * the adequate execution environment ({@link
[2/2] incubator-beam git commit: [BEAM-242] This closes #874
[BEAM-242] This closes #874 Project: http://git-wip-us.apache.org/repos/asf/incubator-beam/repo Commit: http://git-wip-us.apache.org/repos/asf/incubator-beam/commit/be689df0 Tree: http://git-wip-us.apache.org/repos/asf/incubator-beam/tree/be689df0 Diff: http://git-wip-us.apache.org/repos/asf/incubator-beam/diff/be689df0 Branch: refs/heads/master Commit: be689df07530be83c5d01da6d70f1d4e792f54d4 Parents: 8ca6830 9514557 Author: Jean-Baptiste OnofréAuthored: Mon Sep 5 17:25:57 2016 +0200 Committer: Jean-Baptiste Onofré Committed: Mon Sep 5 17:25:57 2016 +0200 -- runners/flink/runner/pom.xml| 2 - .../FlinkPipelineExecutionEnvironment.java | 17 +- .../runners/flink/FlinkPipelineOptions.java | 27 ++-- .../runners/flink/FlinkRunnerRegistrar.java | 6 + .../beam/runners/flink/FlinkRunnerResult.java | 17 +- .../beam/runners/flink/TestFlinkRunner.java | 8 +- .../apache/beam/runners/flink/package-info.java | 22 +++ .../FlinkBatchPipelineTranslator.java | 15 +- .../FlinkBatchTranslationContext.java | 10 +- .../translation/FlinkPipelineTranslator.java| 2 +- .../FlinkStreamingTransformTranslators.java | 5 +- .../flink/translation/TranslationMode.java | 8 +- .../translation/functions/package-info.java | 22 +++ .../runners/flink/translation/package-info.java | 22 +++ .../translation/types/CoderTypeSerializer.java | 2 +- .../types/EncodedValueSerializer.java | 162 ++- .../flink/translation/types/package-info.java | 22 +++ .../utils/SerializedPipelineOptions.java| 2 +- .../flink/translation/utils/package-info.java | 22 +++ .../wrappers/DataOutputViewWrapper.java | 2 +- .../translation/wrappers/package-info.java | 22 +++ .../wrappers/streaming/DoFnOperator.java| 12 +- .../streaming/SingletonKeyedWorkItem.java | 5 + .../streaming/SingletonKeyedWorkItemCoder.java | 14 +- .../wrappers/streaming/WindowDoFnOperator.java | 2 +- .../wrappers/streaming/WorkItemKeySelector.java | 3 +- .../streaming/io/UnboundedFlinkSink.java| 13 +- .../streaming/io/UnboundedFlinkSource.java | 29 ++-- .../streaming/io/UnboundedSocketSource.java | 46 -- .../wrappers/streaming/io/package-info.java | 22 +++ .../wrappers/streaming/package-info.java| 22 +++ .../beam/runners/flink/PipelineOptionsTest.java | 3 + .../beam/runners/flink/WriteSinkITCase.java | 3 +- .../apache/beam/runners/flink/package-info.java | 22 +++ .../streaming/FlinkStateInternalsTest.java | 3 +- .../flink/streaming/GroupByNullKeyTest.java | 6 + .../streaming/TopWikipediaSessionsITCase.java | 2 +- .../streaming/UnboundedSourceWrapperTest.java | 33 ++-- .../runners/flink/streaming/package-info.java | 22 +++ 39 files changed, 490 insertions(+), 189 deletions(-) --
[1/2] incubator-beam git commit: [BEAM-456] Add MongoDbIO
Repository: incubator-beam Updated Branches: refs/heads/master a2c223889 -> 8ca683026 [BEAM-456] Add MongoDbIO Project: http://git-wip-us.apache.org/repos/asf/incubator-beam/repo Commit: http://git-wip-us.apache.org/repos/asf/incubator-beam/commit/4b60e368 Tree: http://git-wip-us.apache.org/repos/asf/incubator-beam/tree/4b60e368 Diff: http://git-wip-us.apache.org/repos/asf/incubator-beam/diff/4b60e368 Branch: refs/heads/master Commit: 4b60e36816df66915061f2c834dabf3ca4ac1b89 Parents: a2c2238 Author: Jean-Baptiste OnofréAuthored: Fri Jul 15 18:44:26 2016 +0200 Committer: Jean-Baptiste Onofré Committed: Sun Sep 4 21:56:55 2016 +0200 -- sdks/java/io/mongodb/pom.xml| 129 + .../apache/beam/sdk/io/mongodb/MongoDbIO.java | 553 +++ .../beam/sdk/io/mongodb/package-info.java | 22 + .../beam/sdk/io/mongodb/MongoDbIOTest.java | 209 +++ .../beam/sdk/io/mongodb/package-info.java | 22 + sdks/java/io/pom.xml| 1 + 6 files changed, 936 insertions(+) -- http://git-wip-us.apache.org/repos/asf/incubator-beam/blob/4b60e368/sdks/java/io/mongodb/pom.xml -- diff --git a/sdks/java/io/mongodb/pom.xml b/sdks/java/io/mongodb/pom.xml new file mode 100644 index 000..60f1d1e --- /dev/null +++ b/sdks/java/io/mongodb/pom.xml @@ -0,0 +1,129 @@ + + +http://maven.apache.org/POM/4.0.0; xmlns:xsi="http://www.w3.org/2001/XMLSchema-instance; xsi:schemaLocation="http://maven.apache.org/POM/4.0.0 http://maven.apache.org/xsd/maven-4.0.0.xsd;> + + 4.0.0 + + +org.apache.beam +beam-sdks-java-io-parent +0.3.0-incubating-SNAPSHOT +../pom.xml + + + beam-sdks-java-io-mongodb + Apache Beam :: SDKs :: Java :: IO :: MongoDB + IO to read and write on MongoDB. + + + + +org.apache.maven.plugins +maven-compiler-plugin + + +org.apache.maven.plugins +maven-source-plugin + + +org.apache.maven.plugins +maven-surefire-plugin + + +org.apache.maven.plugins +maven-jar-plugin + + +org.apache.maven.plugins +maven-checkstyle-plugin + + +org.apache.maven.plugins +maven-javadoc-plugin + + + + + +3.2.2 + + + + + org.apache.beam + beam-sdks-java-core + + + + org.slf4j + slf4j-api + + + + com.google.guava + guava + + + + com.google.code.findbugs + annotations + + + + org.mongodb + mongo-java-driver + ${mongo-java-driver.version} + + + + + de.flapdoodle.embed + de.flapdoodle.embed.mongo + 1.50.1 + test + + + de.flapdoodle.embed + de.flapdoodle.embed.process + 1.50.1 + test + + + junit + junit + test + + + org.slf4j + slf4j-jdk14 + test + + + org.apache.beam + beam-runners-direct-java + ${project.version} + test + + + org.hamcrest + hamcrest-all + test + + + + \ No newline at end of file http://git-wip-us.apache.org/repos/asf/incubator-beam/blob/4b60e368/sdks/java/io/mongodb/src/main/java/org/apache/beam/sdk/io/mongodb/MongoDbIO.java -- diff --git a/sdks/java/io/mongodb/src/main/java/org/apache/beam/sdk/io/mongodb/MongoDbIO.java b/sdks/java/io/mongodb/src/main/java/org/apache/beam/sdk/io/mongodb/MongoDbIO.java new file mode 100644 index 000..7724614 --- /dev/null +++ b/sdks/java/io/mongodb/src/main/java/org/apache/beam/sdk/io/mongodb/MongoDbIO.java @@ -0,0 +1,553 @@ +/* + * 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. + */ +package org.apache.beam.sdk.io.mongodb; + +import com.google.common.annotations.VisibleForTesting; +import com.google.common.base.Preconditions; + +import com.mongodb.BasicDBObject; +import
[2/2] incubator-beam git commit: [BEAM-456] This closes #671
[BEAM-456] This closes #671 Project: http://git-wip-us.apache.org/repos/asf/incubator-beam/repo Commit: http://git-wip-us.apache.org/repos/asf/incubator-beam/commit/8ca68302 Tree: http://git-wip-us.apache.org/repos/asf/incubator-beam/tree/8ca68302 Diff: http://git-wip-us.apache.org/repos/asf/incubator-beam/diff/8ca68302 Branch: refs/heads/master Commit: 8ca683026456c2973a88984cfc718ac8313707ea Parents: a2c2238 4b60e36 Author: Jean-Baptiste OnofréAuthored: Sun Sep 4 21:57:19 2016 +0200 Committer: Jean-Baptiste Onofré Committed: Sun Sep 4 21:57:19 2016 +0200 -- sdks/java/io/mongodb/pom.xml| 129 + .../apache/beam/sdk/io/mongodb/MongoDbIO.java | 553 +++ .../beam/sdk/io/mongodb/package-info.java | 22 + .../beam/sdk/io/mongodb/MongoDbIOTest.java | 209 +++ .../beam/sdk/io/mongodb/package-info.java | 22 + sdks/java/io/pom.xml| 1 + 6 files changed, 936 insertions(+) --
[2/2] incubator-beam git commit: [BEAM-569] This closes #898
[BEAM-569] This closes #898 Project: http://git-wip-us.apache.org/repos/asf/incubator-beam/repo Commit: http://git-wip-us.apache.org/repos/asf/incubator-beam/commit/33d747ef Tree: http://git-wip-us.apache.org/repos/asf/incubator-beam/tree/33d747ef Diff: http://git-wip-us.apache.org/repos/asf/incubator-beam/diff/33d747ef Branch: refs/heads/master Commit: 33d747efa2888961baf7b4d9a9b01ec1b4715c62 Parents: 92451d0 800c098 Author: Jean-Baptiste OnofréAuthored: Tue Aug 30 21:47:31 2016 +0200 Committer: Jean-Baptiste Onofré Committed: Tue Aug 30 21:47:31 2016 +0200 -- .../jms/src/main/java/org/apache/beam/sdk/io/jms/JmsIO.java | 8 ++-- 1 file changed, 2 insertions(+), 6 deletions(-) --
[1/2] incubator-beam git commit: [BEAM-569] Define maxNumRecords default value to Long.MAX_VALUE in JmsIO
Repository: incubator-beam Updated Branches: refs/heads/master 92451d071 -> 33d747efa [BEAM-569] Define maxNumRecords default value to Long.MAX_VALUE in JmsIO Project: http://git-wip-us.apache.org/repos/asf/incubator-beam/repo Commit: http://git-wip-us.apache.org/repos/asf/incubator-beam/commit/800c0987 Tree: http://git-wip-us.apache.org/repos/asf/incubator-beam/tree/800c0987 Diff: http://git-wip-us.apache.org/repos/asf/incubator-beam/diff/800c0987 Branch: refs/heads/master Commit: 800c09870a732a71c897705fbecdedec4a961804 Parents: 92451d0 Author: Jean-Baptiste OnofréAuthored: Sat Aug 27 14:01:34 2016 +0200 Committer: Jean-Baptiste Onofré Committed: Tue Aug 30 21:46:33 2016 +0200 -- .../jms/src/main/java/org/apache/beam/sdk/io/jms/JmsIO.java | 8 ++-- 1 file changed, 2 insertions(+), 6 deletions(-) -- http://git-wip-us.apache.org/repos/asf/incubator-beam/blob/800c0987/sdks/java/io/jms/src/main/java/org/apache/beam/sdk/io/jms/JmsIO.java -- diff --git a/sdks/java/io/jms/src/main/java/org/apache/beam/sdk/io/jms/JmsIO.java b/sdks/java/io/jms/src/main/java/org/apache/beam/sdk/io/jms/JmsIO.java index 29d0c5f..3107aab 100644 --- a/sdks/java/io/jms/src/main/java/org/apache/beam/sdk/io/jms/JmsIO.java +++ b/sdks/java/io/jms/src/main/java/org/apache/beam/sdk/io/jms/JmsIO.java @@ -101,11 +101,11 @@ public class JmsIO { private static final Logger LOG = LoggerFactory.getLogger(JmsIO.class); public static Read read() { -return new Read(); +return new Read(null, null, null, Long.MAX_VALUE, null); } public static Write write() { -return new Write(); +return new Write(null, null, null); } /** @@ -185,8 +185,6 @@ public class JmsIO { protected long maxNumRecords; protected Duration maxReadTime; -private Read() {} - private Read( ConnectionFactory connectionFactory, String queue, @@ -428,8 +426,6 @@ public class JmsIO { return new Write(connectionFactory, queue, topic); } -private Write() {} - private Write(ConnectionFactory connectionFactory, String queue, String topic) { this.connectionFactory = connectionFactory; this.queue = queue;
[1/2] incubator-beam git commit: [BEAM-313] Provide a context for SparkRunner
Repository: incubator-beam Updated Branches: refs/heads/master 676843e04 -> 3666c22cb [BEAM-313] Provide a context for SparkRunner Project: http://git-wip-us.apache.org/repos/asf/incubator-beam/repo Commit: http://git-wip-us.apache.org/repos/asf/incubator-beam/commit/017da7ba Tree: http://git-wip-us.apache.org/repos/asf/incubator-beam/tree/017da7ba Diff: http://git-wip-us.apache.org/repos/asf/incubator-beam/diff/017da7ba Branch: refs/heads/master Commit: 017da7bac3e844ef7391aabbcbaf86c9c99af968 Parents: 676843e Author: Abbass MAROUNIAuthored: Mon Aug 29 13:28:46 2016 +0200 Committer: Abbass MAROUNI Committed: Mon Aug 29 13:28:46 2016 +0200 -- .../runners/spark/SparkPipelineOptions.java | 13 ++ .../apache/beam/runners/spark/SparkRunner.java | 16 ++- .../runners/spark/ProvidedSparkContextTest.java | 138 +++ 3 files changed, 164 insertions(+), 3 deletions(-) -- http://git-wip-us.apache.org/repos/asf/incubator-beam/blob/017da7ba/runners/spark/src/main/java/org/apache/beam/runners/spark/SparkPipelineOptions.java -- diff --git a/runners/spark/src/main/java/org/apache/beam/runners/spark/SparkPipelineOptions.java b/runners/spark/src/main/java/org/apache/beam/runners/spark/SparkPipelineOptions.java index be4f7f0..db6b75c 100644 --- a/runners/spark/src/main/java/org/apache/beam/runners/spark/SparkPipelineOptions.java +++ b/runners/spark/src/main/java/org/apache/beam/runners/spark/SparkPipelineOptions.java @@ -18,11 +18,14 @@ package org.apache.beam.runners.spark; +import com.fasterxml.jackson.annotation.JsonIgnore; + import org.apache.beam.sdk.options.ApplicationNameOptions; import org.apache.beam.sdk.options.Default; import org.apache.beam.sdk.options.Description; import org.apache.beam.sdk.options.PipelineOptions; import org.apache.beam.sdk.options.StreamingOptions; +import org.apache.spark.api.java.JavaSparkContext; /** * Spark runner pipeline options. @@ -49,4 +52,14 @@ public interface SparkPipelineOptions extends PipelineOptions, StreamingOptions, @Default.Boolean(true) Boolean getEnableSparkSinks(); void setEnableSparkSinks(Boolean enableSparkSinks); + + @Description("If the spark runner will be initialized with a provided Spark Context") + @Default.Boolean(false) + boolean getUsesProvidedSparkContext(); + void setUsesProvidedSparkContext(boolean value); + + @Description("Provided Java Spark Context") + @JsonIgnore + JavaSparkContext getProvidedSparkContext(); + void setProvidedSparkContext(JavaSparkContext jsc); } http://git-wip-us.apache.org/repos/asf/incubator-beam/blob/017da7ba/runners/spark/src/main/java/org/apache/beam/runners/spark/SparkRunner.java -- diff --git a/runners/spark/src/main/java/org/apache/beam/runners/spark/SparkRunner.java b/runners/spark/src/main/java/org/apache/beam/runners/spark/SparkRunner.java index fa85a2e..9f1a839 100644 --- a/runners/spark/src/main/java/org/apache/beam/runners/spark/SparkRunner.java +++ b/runners/spark/src/main/java/org/apache/beam/runners/spark/SparkRunner.java @@ -143,9 +143,19 @@ public final class SparkRunner extends PipelineRunner { public EvaluationResult run(Pipeline pipeline) { try { LOG.info("Executing pipeline using the SparkRunner."); - JavaSparkContext jsc = SparkContextFactory.getSparkContext(mOptions.getSparkMaster(), - mOptions.getAppName()); - + JavaSparkContext jsc; + if (mOptions.getUsesProvidedSparkContext()) { +LOG.info("Using a provided Spark Context"); +jsc = mOptions.getProvidedSparkContext(); +if (jsc == null || jsc.sc().isStopped()){ + LOG.error("The provided Spark context " + + jsc + " was not created or was stopped"); + throw new RuntimeException("The provided Spark context was not created or was stopped"); +} + } else { +LOG.info("Creating a new Spark Context"); +jsc = SparkContextFactory.getSparkContext(mOptions.getSparkMaster(), mOptions.getAppName()); + } if (mOptions.isStreaming()) { SparkPipelineTranslator translator = new StreamingTransformTranslator.Translator(new TransformTranslator.Translator()); http://git-wip-us.apache.org/repos/asf/incubator-beam/blob/017da7ba/runners/spark/src/test/java/org/apache/beam/runners/spark/ProvidedSparkContextTest.java -- diff --git a/runners/spark/src/test/java/org/apache/beam/runners/spark/ProvidedSparkContextTest.java b/runners/spark/src/test/java/org/apache/beam/runners/spark/ProvidedSparkContextTest.java new file mode 100644 index 000..cbc5976 --- /dev/null +++
[2/2] incubator-beam git commit: [BEAM-313] This closes #401
[BEAM-313] This closes #401 Project: http://git-wip-us.apache.org/repos/asf/incubator-beam/repo Commit: http://git-wip-us.apache.org/repos/asf/incubator-beam/commit/3666c22c Tree: http://git-wip-us.apache.org/repos/asf/incubator-beam/tree/3666c22c Diff: http://git-wip-us.apache.org/repos/asf/incubator-beam/diff/3666c22c Branch: refs/heads/master Commit: 3666c22cbf06009d97ab39707318aae56c9da907 Parents: 676843e 017da7b Author: Jean-Baptiste OnofréAuthored: Mon Aug 29 13:56:54 2016 +0200 Committer: Jean-Baptiste Onofré Committed: Mon Aug 29 13:56:54 2016 +0200 -- .../runners/spark/SparkPipelineOptions.java | 13 ++ .../apache/beam/runners/spark/SparkRunner.java | 16 ++- .../runners/spark/ProvidedSparkContextTest.java | 138 +++ 3 files changed, 164 insertions(+), 3 deletions(-) --
[GitHub] incubator-beam pull request #898: [BEAM-569] Define maxNumRecords default va...
GitHub user jbonofre opened a pull request: https://github.com/apache/incubator-beam/pull/898 [BEAM-569] Define maxNumRecords default value to Long.MAX_VALUE in JmsIO Be sure to do all of the following to help us incorporate your contribution quickly and easily: - [X] Make sure the PR title is formatted like: `[BEAM-] Description of pull request` - [X] Make sure tests pass via `mvn clean verify`. (Even better, enable Travis-CI on your fork and ensure the whole test matrix passes). - [X] Replace `` in the title with the actual Jira issue number, if there is one. - [X] If this contribution is large, please file an Apache [Individual Contributor License Agreement](https://www.apache.org/licenses/icla.txt). --- You can merge this pull request into a Git repository by running: $ git pull https://github.com/jbonofre/incubator-beam BEAM-569 Alternatively you can review and apply these changes as the patch at: https://github.com/apache/incubator-beam/pull/898.patch To close this pull request, make a commit to your master/trunk branch with (at least) the following in the commit message: This closes #898 commit 086637ae726a28a9cc133d022adb01b11939c7f3 Author: Jean-Baptiste Onofré <jbono...@apache.org> Date: 2016-08-27T12:01:34Z [BEAM-569] Define maxNumRecords default value to Long.MAX_VALUE in JmsIO --- If your project is set up for it, you can reply to this email and have your reply appear on GitHub as well. If your project does not have this feature enabled and wishes so, or if the feature is enabled but not working, please contact infrastructure at infrastruct...@apache.org or file a JIRA ticket with INFRA. ---
[2/2] incubator-beam git commit: [BEAM-294] This closes #884
[BEAM-294] This closes #884 Project: http://git-wip-us.apache.org/repos/asf/incubator-beam/repo Commit: http://git-wip-us.apache.org/repos/asf/incubator-beam/commit/20467832 Tree: http://git-wip-us.apache.org/repos/asf/incubator-beam/tree/20467832 Diff: http://git-wip-us.apache.org/repos/asf/incubator-beam/diff/20467832 Branch: refs/heads/master Commit: 2046783235f0011e6003a2a9254b26aa5c3c27b9 Parents: b21c35d e233e5f Author: Jean-Baptiste OnofréAuthored: Fri Aug 26 09:39:35 2016 +0200 Committer: Jean-Baptiste Onofré Committed: Fri Aug 26 09:39:35 2016 +0200 -- runners/spark/pom.xml| 2 +- .../main/java/org/apache/beam/runners/spark/SparkRunner.java | 2 +- .../java/org/apache/beam/runners/spark/TestSparkRunner.java | 2 +- .../beam/runners/spark/aggregators/NamedAggregators.java | 2 +- .../org/apache/beam/runners/spark/examples/WordCount.java| 2 +- .../runners/spark/io/hadoop/ShardNameTemplateHelper.java | 6 +++--- .../apache/beam/runners/spark/translation/DoFnFunction.java | 2 +- .../runners/spark/translation/SparkPipelineTranslator.java | 2 +- .../beam/runners/spark/translation/SparkRuntimeContext.java | 2 +- .../beam/runners/spark/translation/TransformTranslator.java | 4 ++-- .../translation/streaming/StreamingTransformTranslator.java | 8 .../org/apache/beam/runners/spark/util/BroadcastHelper.java | 4 ++-- .../runners/spark/translation/TransformTranslatorTest.java | 2 +- .../spark/translation/streaming/KafkaStreamingTest.java | 2 +- 14 files changed, 21 insertions(+), 21 deletions(-) --
[1/2] incubator-beam git commit: [BEAM-294] Rename dataflow references to beam
Repository: incubator-beam Updated Branches: refs/heads/master b21c35d1a -> 204678323 [BEAM-294] Rename dataflow references to beam Project: http://git-wip-us.apache.org/repos/asf/incubator-beam/repo Commit: http://git-wip-us.apache.org/repos/asf/incubator-beam/commit/e233e5f6 Tree: http://git-wip-us.apache.org/repos/asf/incubator-beam/tree/e233e5f6 Diff: http://git-wip-us.apache.org/repos/asf/incubator-beam/diff/e233e5f6 Branch: refs/heads/master Commit: e233e5f64d8bfeb5b4da7d96515e939c4bfd8b0e Parents: bfd810f Author: Jean-Baptiste OnofréAuthored: Thu Aug 25 14:32:20 2016 +0200 Committer: Jean-Baptiste Onofré Committed: Fri Aug 26 09:12:36 2016 +0200 -- runners/spark/pom.xml| 2 +- .../main/java/org/apache/beam/runners/spark/SparkRunner.java | 2 +- .../java/org/apache/beam/runners/spark/TestSparkRunner.java | 2 +- .../beam/runners/spark/aggregators/NamedAggregators.java | 2 +- .../org/apache/beam/runners/spark/examples/WordCount.java| 2 +- .../runners/spark/io/hadoop/ShardNameTemplateHelper.java | 6 +++--- .../apache/beam/runners/spark/translation/DoFnFunction.java | 2 +- .../runners/spark/translation/SparkPipelineTranslator.java | 2 +- .../beam/runners/spark/translation/SparkRuntimeContext.java | 2 +- .../beam/runners/spark/translation/TransformTranslator.java | 4 ++-- .../translation/streaming/StreamingTransformTranslator.java | 8 .../org/apache/beam/runners/spark/util/BroadcastHelper.java | 4 ++-- .../runners/spark/translation/TransformTranslatorTest.java | 2 +- .../spark/translation/streaming/KafkaStreamingTest.java | 2 +- 14 files changed, 21 insertions(+), 21 deletions(-) -- http://git-wip-us.apache.org/repos/asf/incubator-beam/blob/e233e5f6/runners/spark/pom.xml -- diff --git a/runners/spark/pom.xml b/runners/spark/pom.xml index a5e99a0..b924cb8 100644 --- a/runners/spark/pom.xml +++ b/runners/spark/pom.xml @@ -330,7 +330,7 @@ - com.google.common http://git-wip-us.apache.org/repos/asf/incubator-beam/blob/e233e5f6/runners/spark/src/main/java/org/apache/beam/runners/spark/SparkRunner.java -- diff --git a/runners/spark/src/main/java/org/apache/beam/runners/spark/SparkRunner.java b/runners/spark/src/main/java/org/apache/beam/runners/spark/SparkRunner.java index 2ce1ff6..fa85a2e 100644 --- a/runners/spark/src/main/java/org/apache/beam/runners/spark/SparkRunner.java +++ b/runners/spark/src/main/java/org/apache/beam/runners/spark/SparkRunner.java @@ -50,7 +50,7 @@ import org.slf4j.LoggerFactory; /** * The SparkRunner translate operations defined on a pipeline to a representation * executable by Spark, and then submitting the job to Spark to be executed. If we wanted to run - * a dataflow pipeline with the default options of a single threaded spark instance in local mode, + * a Beam pipeline with the default options of a single threaded spark instance in local mode, * we would do the following: * * {@code http://git-wip-us.apache.org/repos/asf/incubator-beam/blob/e233e5f6/runners/spark/src/main/java/org/apache/beam/runners/spark/TestSparkRunner.java -- diff --git a/runners/spark/src/main/java/org/apache/beam/runners/spark/TestSparkRunner.java b/runners/spark/src/main/java/org/apache/beam/runners/spark/TestSparkRunner.java index 50ed5f3..376b80f 100644 --- a/runners/spark/src/main/java/org/apache/beam/runners/spark/TestSparkRunner.java +++ b/runners/spark/src/main/java/org/apache/beam/runners/spark/TestSparkRunner.java @@ -28,7 +28,7 @@ import org.apache.beam.sdk.values.POutput; /** * The SparkRunner translate operations defined on a pipeline to a representation executable - * by Spark, and then submitting the job to Spark to be executed. If we wanted to run a dataflow + * by Spark, and then submitting the job to Spark to be executed. If we wanted to run a Beam * pipeline with the default options of a single threaded spark instance in local mode, we would do * the following: * http://git-wip-us.apache.org/repos/asf/incubator-beam/blob/e233e5f6/runners/spark/src/main/java/org/apache/beam/runners/spark/aggregators/NamedAggregators.java -- diff --git a/runners/spark/src/main/java/org/apache/beam/runners/spark/aggregators/NamedAggregators.java b/runners/spark/src/main/java/org/apache/beam/runners/spark/aggregators/NamedAggregators.java index c15e276..e2cd963 100644 ---
[GitHub] incubator-beam pull request #884: [BEAM-294] Rename dataflow references to b...
GitHub user jbonofre opened a pull request: https://github.com/apache/incubator-beam/pull/884 [BEAM-294] Rename dataflow references to beam Be sure to do all of the following to help us incorporate your contribution quickly and easily: - [X] Make sure the PR title is formatted like: `[BEAM-] Description of pull request` - [X] Make sure tests pass via `mvn clean verify`. (Even better, enable Travis-CI on your fork and ensure the whole test matrix passes). - [X] Replace `` in the title with the actual Jira issue number, if there is one. - [X] If this contribution is large, please file an Apache [Individual Contributor License Agreement](https://www.apache.org/licenses/icla.txt). --- You can merge this pull request into a Git repository by running: $ git pull https://github.com/jbonofre/incubator-beam BEAM-294 Alternatively you can review and apply these changes as the patch at: https://github.com/apache/incubator-beam/pull/884.patch To close this pull request, make a commit to your master/trunk branch with (at least) the following in the commit message: This closes #884 commit 229c042a6b179236230fdfa61d5c0550a28e0d3b Author: Jean-Baptiste Onofré <jbono...@apache.org> Date: 2016-08-25T12:32:20Z [BEAM-294] Rename dataflow references to beam --- If your project is set up for it, you can reply to this email and have your reply appear on GitHub as well. If your project does not have this feature enabled and wishes so, or if the feature is enabled but not working, please contact infrastructure at infrastruct...@apache.org or file a JIRA ticket with INFRA. ---
[GitHub] incubator-beam pull request #874: [BEAM-242] Enable checkstyle and fix check...
GitHub user jbonofre opened a pull request: https://github.com/apache/incubator-beam/pull/874 [BEAM-242] Enable checkstyle and fix checkstyle errors in Flink runner Be sure to do all of the following to help us incorporate your contribution quickly and easily: - [X] Make sure the PR title is formatted like: `[BEAM-] Description of pull request` - [X] Make sure tests pass via `mvn clean verify`. (Even better, enable Travis-CI on your fork and ensure the whole test matrix passes). - [X] Replace `` in the title with the actual Jira issue number, if there is one. - [X] If this contribution is large, please file an Apache [Individual Contributor License Agreement](https://www.apache.org/licenses/icla.txt). --- You can merge this pull request into a Git repository by running: $ git pull https://github.com/jbonofre/incubator-beam BEAM-242 Alternatively you can review and apply these changes as the patch at: https://github.com/apache/incubator-beam/pull/874.patch To close this pull request, make a commit to your master/trunk branch with (at least) the following in the commit message: This closes #874 --- If your project is set up for it, you can reply to this email and have your reply appear on GitHub as well. If your project does not have this feature enabled and wishes so, or if the feature is enabled but not working, please contact infrastructure at infrastruct...@apache.org or file a JIRA ticket with INFRA. ---
[1/2] incubator-beam git commit: [BEAM-486] Remove unnecessary mention of Apache v2.0 LICENSE
Repository: incubator-beam Updated Branches: refs/heads/master 65045f98a -> 76928d3bc [BEAM-486] Remove unnecessary mention of Apache v2.0 LICENSE Project: http://git-wip-us.apache.org/repos/asf/incubator-beam/repo Commit: http://git-wip-us.apache.org/repos/asf/incubator-beam/commit/15b7f81f Tree: http://git-wip-us.apache.org/repos/asf/incubator-beam/tree/15b7f81f Diff: http://git-wip-us.apache.org/repos/asf/incubator-beam/diff/15b7f81f Branch: refs/heads/master Commit: 15b7f81f453966bf7c2066862bd639c97b3b0e6d Parents: 65045f9 Author: Dan HalperinAuthored: Mon Jul 25 16:42:23 2016 -0700 Committer: Jean-Baptiste Onofré Committed: Wed Jul 27 21:45:36 2016 +0200 -- NOTICE | 1 - 1 file changed, 1 deletion(-) -- http://git-wip-us.apache.org/repos/asf/incubator-beam/blob/15b7f81f/NOTICE -- diff --git a/NOTICE b/NOTICE index ef729ba..bd3400c 100644 --- a/NOTICE +++ b/NOTICE @@ -9,4 +9,3 @@ Google (http://www.google.com/). This product includes software developed at Google (http://www.google.com/). -Licensed under the Apache v2.0 License.
[2/2] incubator-beam git commit: [BEAM-486] This closes #727
[BEAM-486] This closes #727 Project: http://git-wip-us.apache.org/repos/asf/incubator-beam/repo Commit: http://git-wip-us.apache.org/repos/asf/incubator-beam/commit/76928d3b Tree: http://git-wip-us.apache.org/repos/asf/incubator-beam/tree/76928d3b Diff: http://git-wip-us.apache.org/repos/asf/incubator-beam/diff/76928d3b Branch: refs/heads/master Commit: 76928d3bc43d30c8fdea0847aec828a535e93d9f Parents: 65045f9 15b7f81 Author: Jean-Baptiste OnofréAuthored: Wed Jul 27 21:46:17 2016 +0200 Committer: Jean-Baptiste Onofré Committed: Wed Jul 27 21:46:17 2016 +0200 -- NOTICE | 1 - 1 file changed, 1 deletion(-) --
[1/2] incubator-beam git commit: [BEAM-488] Remove KEYS file
Repository: incubator-beam Updated Branches: refs/heads/master b6c29e6b3 -> 65045f98a [BEAM-488] Remove KEYS file Per discussion, linked in JIRA: > Bundling PGP keys inside a package is worse than worthless â an > attacker can just bundle spoofed keys with a bogus distro! Keys need > to be made available from a highly reliable, separate server: Download > the main package from a mirror, get PGP keys from apache.org, > pgp.mit.edu, etc. and verify. > > The KEYS file within the Beam source tree should be deleted. Project: http://git-wip-us.apache.org/repos/asf/incubator-beam/repo Commit: http://git-wip-us.apache.org/repos/asf/incubator-beam/commit/9e567341 Tree: http://git-wip-us.apache.org/repos/asf/incubator-beam/tree/9e567341 Diff: http://git-wip-us.apache.org/repos/asf/incubator-beam/diff/9e567341 Branch: refs/heads/master Commit: 9e5673419c29b88a4ba5b1f22e62d048280673b6 Parents: b6c29e6 Author: Dan HalperinAuthored: Mon Jul 25 23:24:10 2016 -0700 Committer: Jean-Baptiste Onofré Committed: Wed Jul 27 21:28:50 2016 +0200 -- KEYS | 141 -- 1 file changed, 141 deletions(-) -- http://git-wip-us.apache.org/repos/asf/incubator-beam/blob/9e567341/KEYS -- diff --git a/KEYS b/KEYS deleted file mode 100644 index 1f14625..000 --- a/KEYS +++ /dev/null @@ -1,141 +0,0 @@ -This file contains the PGP keys of various developers. - -Users: pgp < KEYS - gpg --import KEYS -Developers: -pgp -kxa and append it to this file. -(pgpk -ll && pgpk -xa ) >> this file. -(gpg --list-sigs - && gpg --armor --export ) >> this file. - - -pub 4096R/C8282E76 2009-09-08 -uid Jean-Baptiste Onofré -sig 3C8282E76 2009-09-08 Jean-Baptiste Onofré -sub 4096R/9F043BBC 2009-09-08 -sig C8282E76 2009-09-08 Jean-Baptiste Onofré - --BEGIN PGP PUBLIC KEY BLOCK- -Version: GnuPG v1 - -mQINBEqmJkEBEADAAMOjOidXzoyK4FK9WhhRg2EEGX1gm5lK8PpJtk68Fqmz6xvv -N8VJXMIJUgeD7M35zZSQUWJY43xEU8Yfn6oLL0KR0dIqVOclxE+7G8vxXFcIbRE9 -ziZFp7Z5yzsdzjiIzXv5MVQMczcAAMev/i0BnjiRy5Cg+k6kHXVpu/Gsn05JKPaG -s7ZcfSxpboyS99MVKQvoFLE5Z/Shh4gFJn2rFInqK5EgVpoZbVyysF52nx0dti/e -O0NjraQkrEDBWvsPt3cYZA0oP1gWiZiRvOLfAFIarf3poMDyoWBIwnbqb3Msv09j -yDAmcGq9wsD3alHFHcRIiJl5SzFUStml1d5x/BvUl/Xc5VfHPi2ObKF3xOPGkyTf -aZ6mYFLaRCAJ0v2MPW+4/grDXKsP8n8xPbE2VQvHBpxaZklD7q4Omn2d+m2sUOLX -NRUo4n29NyfowAffBYl7ZqrYBBodR9YngWC9LpgM+APHyiw3HzauZ94bGy5Of3+L -Yu6/riDcP4OXF6r6IH6KIsVqIkv5xzq7OGxxXmlhWg8ifNPLq5yNRccS0nWXc5BD -/9q06ta/ceQGNkXL327XPuZC+lstWGAa4dKEosRDgcO0Pv2j2a3h8W8oHyxF+gEe -O+9s0mGdQFxNiEA+JyeKCg+jvfx9Hv/2Syrlert76NEkfbaTFA7BJ4c3EQARAQAB -tCtKZWFuLUJhcHRpc3RlIE9ub2Zyw6kgPGpib25vZnJlQGFwYWNoZS5vcmc+iQI2 -BBMBAgAgBQJKpiZBAhsDBgsJCAcDAgQVAggDBBYCAwECHgECF4AACgkQv/LuQsgo -LnboyRAAguqFIpiKkCCR6TR0Y5UQDFhgEMhBreQKCEW0czbGoFnxfULV9H1kJRSB -Vt0knecGaYS340WEmz4B7BMpkBCgaszgn66+fhacZTBd+Aff1k2lbhdMgdBvlPcm -q9vFGtbE515j9bPHzsPRJ2wFWd6ot9wXiLD3RJLV6c7L3Egstu3qTp0tEoFHrQps -qskGBl+mahhMyz3BUDlusavB0Y0tb6hhXCR79ErhjQrTgU947isztYWpgJlA40lx -DW0hskZWbuGNXjxUJvTT3pKiYUN32WG+2CDNYHceuhsfRLxO/Wb4BKwwDaHWAlH9 -d5F9/vhdPObSv5GQbuUtmCEzeqADUd65jLLM7WSlvRJ+i4m0/TTeP8y4NfxlVbBP -WuYrQW4gPmDKEDNvEec6PH6hhBfMLJz3M6o4huwLp2kQrq6wSTMDGIoxOLP0ae3c -BMIuFM5EavLDJmuATUIWWyZt/c7mmAOOh5TGcFWTugnJ6l4FllOrFPiWyFsjMn+U -zzzaeSkYmq/xZYxjRTdWjK5Zb5rbVuCx/q5VF9Awdy4EM6UXhaqWo06VyjWNOJ86 -wgres4+bVldB7+TiVi9iO6n80WNlPgIaQJlLc+FRsld4Er21kdXreX5doxFD5Iue -S4y/pLwftHfx1xxj+p2jPJ49Hb0ddNr+XrsrO5txing2pNJgfH65Ag0ESqYmQQEQ -AKPoXgIIKnyJiPvks7xBV+FqJPecVAx3SSlLyTfsh/jBat9QLd4hsfiZcv1ANZHB -n4qDeGlsmJ6uDGv8wnUZQ2Im8Heje1h7dKeLNpNnxfBS9gn6e2bXKhAsJGUE7gip -qVfijFnEY0Vj6Tztzq+Wyqg2Gbz+bJZMo1JVQiaAYyQeQlrOcoZcQHsA/Ol+y48h -Le36A1TSIPMOSI4ZAZXkqxXAumEaMaz82EvV8KDH7Ijr23Y0wZjEUJ+dJQM9ssuE -f9GMLIuCbmM/CJ5MCCwepGJd52ymllvgJTHC7B+BY/jKNMWHwAsMJ1oWcPlLzFQI -Bmyy5RjKoMifzaoSo/hTWkiwcL2Vc+qU3b3/2eUtnCnBB/nkrZkJNNc+OV5YGBSP -vNPaN43Gvjbvborv4PBvt7QhVjZYQemtXO2sWx1XWSFsucD2K4kJ8ipNWxVgIqDu -J8SJOnGigX9hMpsZ2HVAwOeKP/jI90J3voKrCPLaKcL1Ip+b28k0aj7kl44YJqw4 -5pbRSx/v73bH4uleQiXSW+JczA+KLw7hX3tOWJEnLS2+Ig9sNUKYGZOg0nw613bN -fZy8Cbx/UkT10Lznx9FW6MedGyJPYT4MJMMh/PnnsWv50jFnfu2rtnRXEOUXwujL -fwrmCYbXHgE3Ka+fmRz8HxsyTmtqIHtPixw8RoqfoFfxABEBAAGJAh8EGAECAAkF -AkqmJkECGwwACgkQv/LuQsgoLnb8AQ/+POsLFdqNqSKfwBXp1YOIEjNdbVjysQc6 -zC6LlMJXNSxAmUmol2g9bJYh9LdpvOTU3gfFgIanaGytC75U7/NOl0zEsN4IU18j -CLBNaD5/Or1ciQ3CVrID/lPO8s0Hm0/cUPreEjJPPrrPbXG+i9bweg3Dtfy3+WQl -PhfpvgudwtUjB3st2gztYipkUhmrH+STbbJZVJN5ZNL8mOoM5M2wGS+9VweOWbKe -z0QeZ9hIPyQNMzTn1xlvRUVNTu8fz2FGvumrd+zgzYcpTE5VpFkOxxUayr3aWXSf -Cak+HH0WjUDWc9/lJR4dVpwdjLonJfiC70W07J4CnNodYwnPUaGKTVYq3pvQzAPw
[2/2] incubator-beam git commit: [BEAM-488] This closes #732
[BEAM-488] This closes #732 Project: http://git-wip-us.apache.org/repos/asf/incubator-beam/repo Commit: http://git-wip-us.apache.org/repos/asf/incubator-beam/commit/65045f98 Tree: http://git-wip-us.apache.org/repos/asf/incubator-beam/tree/65045f98 Diff: http://git-wip-us.apache.org/repos/asf/incubator-beam/diff/65045f98 Branch: refs/heads/master Commit: 65045f98a09661e485389dddc5fda3f51825bf31 Parents: b6c29e6 9e56734 Author: Jean-Baptiste OnofréAuthored: Wed Jul 27 21:30:16 2016 +0200 Committer: Jean-Baptiste Onofré Committed: Wed Jul 27 21:30:16 2016 +0200 -- KEYS | 141 -- 1 file changed, 141 deletions(-) --
[1/2] incubator-beam git commit: [BEAM-487] Update README.md: add DISCLAIMER, incubating, minor fixes
Repository: incubator-beam Updated Branches: refs/heads/master b1f36df2d -> 4c0e11e50 [BEAM-487] Update README.md: add DISCLAIMER, incubating, minor fixes Project: http://git-wip-us.apache.org/repos/asf/incubator-beam/repo Commit: http://git-wip-us.apache.org/repos/asf/incubator-beam/commit/9329d2e5 Tree: http://git-wip-us.apache.org/repos/asf/incubator-beam/tree/9329d2e5 Diff: http://git-wip-us.apache.org/repos/asf/incubator-beam/diff/9329d2e5 Branch: refs/heads/master Commit: 9329d2e5d3bfed480d3343e5db73c360dfffe38f Parents: b1f36df Author: Dan HalperinAuthored: Mon Jul 25 23:30:44 2016 -0700 Committer: Jean-Baptiste Onofré Committed: Wed Jul 27 19:11:13 2016 +0200 -- README.md | 22 +- 1 file changed, 17 insertions(+), 5 deletions(-) -- http://git-wip-us.apache.org/repos/asf/incubator-beam/blob/9329d2e5/README.md -- diff --git a/README.md b/README.md index 798c12b..e72aec2 100644 --- a/README.md +++ b/README.md @@ -17,10 +17,22 @@ under the License. --> -# Apache Beam +# Apache Beam (incubating) [Apache Beam](http://beam.incubator.apache.org) is a unified model for defining both batch and streaming data-parallel processing pipelines, as well as a set of language-specific SDKs for constructing pipelines and Runners for executing them on distributed processing backends like [Apache Spark](http://spark.apache.org/), [Apache Flink](http://flink.apache.org), and [Google Cloud Dataflow](http://cloud.google.com/dataflow). +``` +Apache Beam is an effort undergoing incubation at the Apache Software +Foundation (ASF), sponsored by the Apache Incubator PMC. + +Incubation is required of all newly accepted projects until a further review +indicates that the infrastructure, communications, and decision making process +have stabilized in a manner consistent with other successful ASF projects. + +While incubation status is not necessarily a reflection of the completeness +or stability of the code, it does indicate that the project has yet to be +fully endorsed by the ASF. +``` ## Status @@ -59,7 +71,7 @@ Beam supports multiple language specific SDKs for writing pipelines against the Currently, this repository contains the Beam Java SDK, which is in the process of evolving from the [Dataflow Java SDK](https://github.com/GoogleCloudPlatform/DataflowJavaSDK). The [Dataflow Python SDK](https://github.com/GoogleCloudPlatform/DataflowPythonSDK) will also become part of Beam in the near future. -Have ideas for new SDKs or DSLs? See the [Jira](https://issues.apache.org/jira/browse/BEAM/component/12328909/). +Have ideas for new SDKs or DSLs? See the [JIRA](https://issues.apache.org/jira/browse/BEAM/component/12328909/). ### Runners @@ -67,11 +79,11 @@ Have ideas for new SDKs or DSLs? See the [Jira](https://issues.apache.org/jira/b Beam supports executing programs on multiple distributed processing backends through PipelineRunners. Currently, the following PipelineRunners are available: - The `DirectRunner` runs the pipeline on your local machine. -- The `DataflowPipelineRunner` submits the pipeline to the [Google Cloud Dataflow](http://cloud.google.com/dataflow/). +- The `DataflowRunner` submits the pipeline to the [Google Cloud Dataflow](http://cloud.google.com/dataflow/). - The `FlinkRunner` runs the pipeline on an Apache Flink cluster. The code has been donated from [dataArtisans/flink-dataflow](https://github.com/dataArtisans/flink-dataflow) and is now part of Beam. - The `SparkRunner` runs the pipeline on an Apache Spark cluster. The code has been donated from [cloudera/spark-dataflow](https://github.com/cloudera/spark-dataflow) and is now part of Beam. -Have ideas for new Runners? See the [Jira](https://issues.apache.org/jira/browse/BEAM/component/12328916/). +Have ideas for new Runners? See the [JIRA](https://issues.apache.org/jira/browse/BEAM/component/12328916/). ## Getting Started @@ -92,7 +104,7 @@ To get involved in Apache Beam: * [Subscribe](mailto:user-subscr...@beam.incubator.apache.org) or [mail](mailto:u...@beam.incubator.apache.org) the [u...@beam.incubator.apache.org](http://mail-archives.apache.org/mod_mbox/incubator-beam-user/) list. * [Subscribe](mailto:dev-subscr...@beam.incubator.apache.org) or [mail](mailto:d...@beam.incubator.apache.org) the [d...@beam.incubator.apache.org](http://mail-archives.apache.org/mod_mbox/incubator-beam-dev/) list. -* Report issues on [Jira](https://issues.apache.org/jira/browse/BEAM). +* Report issues on [JIRA](https://issues.apache.org/jira/browse/BEAM). ## More Information
[GitHub] incubator-beam pull request #402: [BEAM-316] Add file scheme support in Text...
Github user jbonofre closed the pull request at: https://github.com/apache/incubator-beam/pull/402 --- If your project is set up for it, you can reply to this email and have your reply appear on GitHub as well. If your project does not have this feature enabled and wishes so, or if the feature is enabled but not working, please contact infrastructure at infrastruct...@apache.org or file a JIRA ticket with INFRA. ---
[GitHub] incubator-beam pull request #671: [BEAM-456] Add MongoDbIO
GitHub user jbonofre opened a pull request: https://github.com/apache/incubator-beam/pull/671 [BEAM-456] Add MongoDbIO Be sure to do all of the following to help us incorporate your contribution quickly and easily: - [X] Make sure the PR title is formatted like: `[BEAM-] Description of pull request` - [X] Make sure tests pass via `mvn clean verify`. (Even better, enable Travis-CI on your fork and ensure the whole test matrix passes). - [X] Replace `` in the title with the actual Jira issue number, if there is one. - [X] If this contribution is large, please file an Apache [Individual Contributor License Agreement](https://www.apache.org/licenses/icla.txt). --- This is the first "skeleton" for the MongoDbIO. TODO: - find a way to implement estimated size and split in the source - check if it makes sense to manipulate `PCollection` of `Document` or `DBObject` instead of JSON `String`. You can merge this pull request into a Git repository by running: $ git pull https://github.com/jbonofre/incubator-beam BEAM-456-MONGODB Alternatively you can review and apply these changes as the patch at: https://github.com/apache/incubator-beam/pull/671.patch To close this pull request, make a commit to your master/trunk branch with (at least) the following in the commit message: This closes #671 commit 67c977c3508ec9d66c43ae86f4301554c20ff90f Author: Jean-Baptiste Onofré <jbono...@apache.org> Date: 2016-07-15T16:44:26Z [BEAM-456] Add MongoDbIO --- If your project is set up for it, you can reply to this email and have your reply appear on GitHub as well. If your project does not have this feature enabled and wishes so, or if the feature is enabled but not working, please contact infrastructure at infrastruct...@apache.org or file a JIRA ticket with INFRA. ---
[GitHub] incubator-beam pull request #372: [BEAM-242] Enable and fix checkstyle on Fl...
Github user jbonofre closed the pull request at: https://github.com/apache/incubator-beam/pull/372 --- If your project is set up for it, you can reply to this email and have your reply appear on GitHub as well. If your project does not have this feature enabled and wishes so, or if the feature is enabled but not working, please contact infrastructure at infrastruct...@apache.org or file a JIRA ticket with INFRA. ---
[1/4] incubator-beam git commit: [BEAM-338] Cleanup Spark runner test resources to avoid notice update
Repository: incubator-beam Updated Branches: refs/heads/master 1685a6626 -> a7689466d http://git-wip-us.apache.org/repos/asf/incubator-beam/blob/2d16edd7/runners/spark/src/test/resources/pg2264.txt -- diff --git a/runners/spark/src/test/resources/pg2264.txt b/runners/spark/src/test/resources/pg2264.txt deleted file mode 100644 index 477b8ee..000 --- a/runners/spark/src/test/resources/pg2264.txt +++ /dev/null @@ -1,3667 +0,0 @@ -***The Project Gutenberg's Etext of Shakespeare's First Folio*** -The Tragedie of Macbeth* - -This is our 3rd edition of most of these plays. See the index. - - -Copyright laws are changing all over the world, be sure to check -the copyright laws for your country before posting these files!! - -Please take a look at the important information in this header. -We encourage you to keep this file on your own disk, keeping an -electronic path open for the next readers. Do not remove this. - - -**Welcome To The World of Free Plain Vanilla Electronic Texts** - -**Etexts Readable By Both Humans and By Computers, Since 1971** - -*These Etexts Prepared By Hundreds of Volunteers and Donations* - -Information on contacting Project Gutenberg to get Etexts, and -further information is included below. We need your donations. - - -The Tragedie of Macbeth - -by William Shakespeare - -July, 2000 [Etext #2264] - - -***The Project Gutenberg's Etext of Shakespeare's First Folio*** -The Tragedie of Macbeth* - -*This file should be named 0ws3410.txt or 0ws3410.zip** - -Corrected EDITIONS of our etexts get a new NUMBER, 0ws3411.txt -VERSIONS based on separate sources get new LETTER, 0ws3410a.txt - - -Project Gutenberg Etexts are usually created from multiple editions, -all of which are in the Public Domain in the United States, unless a -copyright notice is included. Therefore, we usually do NOT keep any -of these books in compliance with any particular paper edition. - - -We are now trying to release all our books one month in advance -of the official release dates, leaving time for better editing. - -Please note: neither this list nor its contents are final till -midnight of the last day of the month of any such announcement. -The official release date of all Project Gutenberg Etexts is at -Midnight, Central Time, of the last day of the stated month. A -preliminary version may often be posted for suggestion, comment -and editing by those who wish to do so. To be sure you have an -up to date first edition [x10x.xxx] please check file sizes -in the first week of the next month. Since our ftp program has -a bug in it that scrambles the date [tried to fix and failed] a -look at the file size will have to do, but we will try to see a -new copy has at least one byte more or less. - - -Information about Project Gutenberg (one page) - -We produce about two million dollars for each hour we work. The -time it takes us, a rather conservative estimate, is fifty hours -to get any etext selected, entered, proofread, edited, copyright -searched and analyzed, the copyright letters written, etc. This -projected audience is one hundred million readers. If our value -per text is nominally estimated at one dollar then we produce $2 -million dollars per hour this year as we release thirty-six text -files per month, or 432 more Etexts in 1999 for a total of 2000+ -If these reach just 10% of the computerized population, then the -total should reach over 200 billion Etexts given away this year. - -The Goal of Project Gutenberg is to Give Away One Trillion Etext -Files by December 31, 2001. [10,000 x 100,000,000 = 1 Trillion] -This is ten thousand titles each to one hundred million readers, -which is only ~5% of the present number of computer users. - -At our revised rates of production, we will reach only one-third -of that goal by the end of 2001, or about 3,333 Etexts unless we -manage to get some real funding; currently our funding is mostly -from Michael Hart's salary at Carnegie-Mellon University, and an -assortment of sporadic gifts; this salary is only good for a few -more years, so we are looking for something to replace it, as we -don't want Project Gutenberg to be so dependent on one person. - -We need your donations more than ever! - - -All donations should be made to "Project Gutenberg/CMU": and are -tax deductible to the extent allowable by law. (CMU = Carnegie- -Mellon University). - -For these and other matters, please mail to: - -Project Gutenberg -P. O. Box 2782 -Champaign, IL 61825 - -When all other email fails. . .try our Executive Director: -Michael S. Hart-h...@pobox.com forwards to h...@prairienet.org and archive.org -if your mail bounces from archive.org, I will still see it, if -it bounces from prairienet.org, better resend later on. . . . - -We would prefer to send you this information by email. -
[2/4] incubator-beam git commit: [BEAM-338] Cleanup Spark runner test resources to avoid notice update
http://git-wip-us.apache.org/repos/asf/incubator-beam/blob/2d16edd7/runners/spark/src/test/resources/pg1112.txt -- diff --git a/runners/spark/src/test/resources/pg1112.txt b/runners/spark/src/test/resources/pg1112.txt deleted file mode 100644 index 81ee6b8..000 --- a/runners/spark/src/test/resources/pg1112.txt +++ /dev/null @@ -1,4853 +0,0 @@ -The Project Gutenberg EBook of Romeo and Juliet, by William Shakespeare - -This eBook is for the use of anyone anywhere at no cost and with -almost no restrictions whatsoever. You may copy it, give it away or -re-use it under the terms of the Project Gutenberg License included -with this eBook or online at www.gutenberg.org/license - - -Title: Romeo and Juliet - -Author: William Shakespeare - -Posting Date: May 25, 2012 [EBook #1112] -Release Date: November, 1997 [Etext #1112] - -Language: English - - -*** START OF THIS PROJECT GUTENBERG EBOOK ROMEO AND JULIET *** - - - - - - - - - - - - - -*Project Gutenberg is proud to cooperate with The World Library* -in the presentation of The Complete Works of William Shakespeare -for your reading for education and entertainment. HOWEVER, THIS -IS NEITHER SHAREWARE NOR PUBLIC DOMAIN. . .AND UNDER THE LIBRARY -OF THE FUTURE CONDITIONS OF THIS PRESENTATION. . .NO CHARGES MAY -BE MADE FOR *ANY* ACCESS TO THIS MATERIAL. YOU ARE ENCOURAGED!! -TO GIVE IT AWAY TO ANYONE YOU LIKE, BUT NO CHARGES ARE ALLOWED!! - - - - -The Complete Works of William Shakespeare - -The Tragedy of Romeo and Juliet - -The Library of the Future Complete Works of William Shakespeare -Library of the Future is a TradeMark (TM) of World Library Inc. - - -<> - - - - -1595 - -THE TRAGEDY OF ROMEO AND JULIET - -by William Shakespeare - - - -Dramatis Personae - - Chorus. - - - Escalus, Prince of Verona. - - Paris, a young Count, kinsman to the Prince. - - Montague, heads of two houses at variance with each other. - - Capulet, heads of two houses at variance with each other. - - An old Man, of the Capulet family. - - Romeo, son to Montague. - - Tybalt, nephew to Lady Capulet. - - Mercutio, kinsman to the Prince and friend to Romeo. - - Benvolio, nephew to Montague, and friend to Romeo - - Tybalt, nephew to Lady Capulet. - - Friar Laurence, Franciscan. - - Friar John, Franciscan. - - Balthasar, servant to Romeo. - - Abram, servant to Montague. - - Sampson, servant to Capulet. - - Gregory, servant to Capulet. - - Peter, servant to Juliet's nurse. - - An Apothecary. - - Three Musicians. - - An Officer. - - - Lady Montague, wife to Montague. - - Lady Capulet, wife to Capulet. - - Juliet, daughter to Capulet. - - Nurse to Juliet. - - - Citizens of Verona; Gentlemen and Gentlewomen of both houses; -Maskers, Torchbearers, Pages, Guards, Watchmen, Servants, and -Attendants. - -SCENE.--Verona; Mantua. - - - -THE PROLOGUE - -Enter Chorus. - - - Chor. Two households, both alike in dignity, -In fair Verona, where we lay our scene, -From ancient grudge break to new mutiny, -Where civil blood makes civil hands unclean. -From forth the fatal loins of these two foes -A pair of star-cross'd lovers take their life; -Whose misadventur'd piteous overthrows -Doth with their death bury their parents' strife. -The fearful passage of their death-mark'd love, -And the continuance of their parents' rage, -Which, but their children's end, naught could remove, -Is now the two hours' traffic of our stage; -The which if you with patient ears attend, -What here shall miss, our toil shall strive to mend. - [Exit.] - - - - -ACT I. Scene I. -Verona. A public place. - -Enter Sampson and Gregory (with swords and bucklers) of the house -of Capulet. - - - Samp. Gregory, on my word, we'll not carry coals. - - Greg. No, for then we should be colliers. - - Samp. I mean, an we be in choler, we'll draw. - - Greg. Ay, while you live, draw your neck out of collar. - - Samp. I strike quickly, being moved. - - Greg. But thou art not quickly moved to strike. - - Samp. A dog of the house of Montague moves me. - - Greg. To move is to stir, and to be valiant is to stand. -Therefore, if thou art moved, thou runn'st away. - - Samp. A dog of that house shall move me to stand. I will take -the wall of any man or maid of Montague's. - - Greg. That shows thee a weak slave; for the weakest goes to the -wall. - - Samp. 'Tis true; and therefore women, being the weaker vessels, -are ever thrust to the wall. Therefore I will push Montague's men -from the wall and thrust his maids to the wall. - - Greg. The quarrel is between our masters and us their men. - - Samp. 'Tis all one. I will show myself a tyrant. When I have -fought with the men, I will be cruel with the maids- I will cut off -their
[3/4] incubator-beam git commit: [BEAM-338] Cleanup Spark runner test resources to avoid notice update
[BEAM-338] Cleanup Spark runner test resources to avoid notice update Project: http://git-wip-us.apache.org/repos/asf/incubator-beam/repo Commit: http://git-wip-us.apache.org/repos/asf/incubator-beam/commit/2d16edd7 Tree: http://git-wip-us.apache.org/repos/asf/incubator-beam/tree/2d16edd7 Diff: http://git-wip-us.apache.org/repos/asf/incubator-beam/diff/2d16edd7 Branch: refs/heads/master Commit: 2d16edd73f1e53f160135120c6e71b4f7a6e9196 Parents: 1685a66 Author: Jean-Baptiste OnofréAuthored: Sun Jul 3 22:20:27 2016 +0200 Committer: Jean-Baptiste Onofré Committed: Tue Jul 12 17:00:13 2016 +0200 -- runners/spark/src/test/resources/person.avsc | 25 +- runners/spark/src/test/resources/pg1112.txt | 4853 - runners/spark/src/test/resources/pg2264.txt | 3667 3 files changed, 14 insertions(+), 8531 deletions(-) -- http://git-wip-us.apache.org/repos/asf/incubator-beam/blob/2d16edd7/runners/spark/src/test/resources/person.avsc -- diff --git a/runners/spark/src/test/resources/person.avsc b/runners/spark/src/test/resources/person.avsc index c20797d..d480c84 100644 --- a/runners/spark/src/test/resources/person.avsc +++ b/runners/spark/src/test/resources/person.avsc @@ -1,19 +1,22 @@ /* - * Copyright (c) 2014, Cloudera, Inc. All Rights Reserved. + * 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 * - * Cloudera, Inc. 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 * - * http://www.apache.org/licenses/LICENSE-2.0 - * - * This software 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. + * 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. */ { -"namespace": "com.cloudera.dataflow.spark.test", +"namespace": "org.apache.beam.runners.spark.test", "name": "Person", "type": "record", "fields": [
[4/4] incubator-beam git commit: [BEAM-338] This closes #586
[BEAM-338] This closes #586 Project: http://git-wip-us.apache.org/repos/asf/incubator-beam/repo Commit: http://git-wip-us.apache.org/repos/asf/incubator-beam/commit/a7689466 Tree: http://git-wip-us.apache.org/repos/asf/incubator-beam/tree/a7689466 Diff: http://git-wip-us.apache.org/repos/asf/incubator-beam/diff/a7689466 Branch: refs/heads/master Commit: a7689466d3639f55c27545c16c91a68c7f830063 Parents: 1685a66 2d16edd Author: Jean-Baptiste OnofréAuthored: Tue Jul 12 17:19:36 2016 +0200 Committer: Jean-Baptiste Onofré Committed: Tue Jul 12 17:19:36 2016 +0200 -- runners/spark/src/test/resources/person.avsc | 25 +- runners/spark/src/test/resources/pg1112.txt | 4853 - runners/spark/src/test/resources/pg2264.txt | 3667 3 files changed, 14 insertions(+), 8531 deletions(-) --
[1/2] incubator-beam git commit: [BEAM-357] Fix build on Windows
Repository: incubator-beam Updated Branches: refs/heads/master 1a5dd59f0 -> 3bb78cb8e [BEAM-357] Fix build on Windows Project: http://git-wip-us.apache.org/repos/asf/incubator-beam/repo Commit: http://git-wip-us.apache.org/repos/asf/incubator-beam/commit/dc2532a4 Tree: http://git-wip-us.apache.org/repos/asf/incubator-beam/tree/dc2532a4 Diff: http://git-wip-us.apache.org/repos/asf/incubator-beam/diff/dc2532a4 Branch: refs/heads/master Commit: dc2532a40950c8903b3c12b3977756016cb378e5 Parents: 1a5dd59 Author: Romain manni-BucauAuthored: Wed Jun 22 10:42:45 2016 +0200 Committer: Jean-Baptiste Onofré Committed: Wed Jul 6 19:05:29 2016 +0200 -- .../apache/beam/runners/flink/WriteSinkITCase.java| 14 +- .../src/main/resources/beam/checkstyle.xml| 5 - .../java/org/apache/beam/sdk/io/FileBasedSink.java| 7 ++- sdks/java/maven-archetypes/starter/pom.xml| 6 ++ 4 files changed, 29 insertions(+), 3 deletions(-) -- http://git-wip-us.apache.org/repos/asf/incubator-beam/blob/dc2532a4/runners/flink/runner/src/test/java/org/apache/beam/runners/flink/WriteSinkITCase.java -- diff --git a/runners/flink/runner/src/test/java/org/apache/beam/runners/flink/WriteSinkITCase.java b/runners/flink/runner/src/test/java/org/apache/beam/runners/flink/WriteSinkITCase.java index 36d3aef..f1d9097 100644 --- a/runners/flink/runner/src/test/java/org/apache/beam/runners/flink/WriteSinkITCase.java +++ b/runners/flink/runner/src/test/java/org/apache/beam/runners/flink/WriteSinkITCase.java @@ -35,6 +35,7 @@ import org.apache.flink.core.fs.Path; import org.apache.flink.test.util.JavaProgramTestBase; import java.io.File; +import java.io.IOException; import java.io.PrintWriter; import java.net.URI; @@ -53,7 +54,7 @@ public class WriteSinkITCase extends JavaProgramTestBase { @Override protected void preSubmit() throws Exception { -resultPath = getTempDirPath("result"); +resultPath = getTempDirPath("result-" + System.nanoTime()); } @Override @@ -66,6 +67,17 @@ public class WriteSinkITCase extends JavaProgramTestBase { runProgram(resultPath); } + @Override + public void stopCluster() throws Exception { +try { + super.stopCluster(); +} catch (final IOException ioe) { + if (ioe.getMessage().startsWith("Unable to delete file")) { +// that's ok for the test itself, just the OS playing with us on cleanup phase + } +} + } + private static void runProgram(String resultPath) { Pipeline p = FlinkTestPipeline.createForBatch(); http://git-wip-us.apache.org/repos/asf/incubator-beam/blob/dc2532a4/sdks/java/build-tools/src/main/resources/beam/checkstyle.xml -- diff --git a/sdks/java/build-tools/src/main/resources/beam/checkstyle.xml b/sdks/java/build-tools/src/main/resources/beam/checkstyle.xml index 311f599..63bab09 100644 --- a/sdks/java/build-tools/src/main/resources/beam/checkstyle.xml +++ b/sdks/java/build-tools/src/main/resources/beam/checkstyle.xml @@ -29,7 +29,10 @@ page at http://checkstyle.sourceforge.net/config.html --> - + + + + http://git-wip-us.apache.org/repos/asf/incubator-beam/blob/dc2532a4/sdks/java/core/src/main/java/org/apache/beam/sdk/io/FileBasedSink.java -- diff --git a/sdks/java/core/src/main/java/org/apache/beam/sdk/io/FileBasedSink.java b/sdks/java/core/src/main/java/org/apache/beam/sdk/io/FileBasedSink.java index 02fc63a..8246148 100644 --- a/sdks/java/core/src/main/java/org/apache/beam/sdk/io/FileBasedSink.java +++ b/sdks/java/core/src/main/java/org/apache/beam/sdk/io/FileBasedSink.java @@ -38,6 +38,7 @@ import com.google.common.collect.Ordering; import org.slf4j.Logger; import org.slf4j.LoggerFactory; +import java.io.File; import java.io.IOException; import java.io.Serializable; import java.nio.channels.WritableByteChannel; @@ -648,7 +649,11 @@ public abstract class FileBasedSink extends Sink { private void copyOne(String source, String destination) throws IOException { try { // Copy the source file, replacing the existing destination. -Files.copy(Paths.get(source), Paths.get(destination), StandardCopyOption.REPLACE_EXISTING); +// Paths.get(x) will not work on win cause of the ":" after the drive letter +Files.copy( +new File(source).toPath(), +new File(destination).toPath(), +StandardCopyOption.REPLACE_EXISTING); } catch (NoSuchFileException e) { LOG.debug("{} does not exist.", source); // Suppress exception if file does
[2/2] incubator-beam git commit: [BEAM-357] This closes #519
[BEAM-357] This closes #519 Project: http://git-wip-us.apache.org/repos/asf/incubator-beam/repo Commit: http://git-wip-us.apache.org/repos/asf/incubator-beam/commit/3bb78cb8 Tree: http://git-wip-us.apache.org/repos/asf/incubator-beam/tree/3bb78cb8 Diff: http://git-wip-us.apache.org/repos/asf/incubator-beam/diff/3bb78cb8 Branch: refs/heads/master Commit: 3bb78cb8ec11b77f6b6821fad8a614b1699c0f97 Parents: 1a5dd59 dc2532a Author: Jean-Baptiste OnofréAuthored: Wed Jul 6 19:19:05 2016 +0200 Committer: Jean-Baptiste Onofré Committed: Wed Jul 6 19:19:05 2016 +0200 -- .../apache/beam/runners/flink/WriteSinkITCase.java| 14 +- .../src/main/resources/beam/checkstyle.xml| 5 - .../java/org/apache/beam/sdk/io/FileBasedSink.java| 7 ++- sdks/java/maven-archetypes/starter/pom.xml| 6 ++ 4 files changed, 29 insertions(+), 3 deletions(-) --
[GitHub] incubator-beam pull request #592: [BEAM-245] Add Cassandra IO
GitHub user jbonofre opened a pull request: https://github.com/apache/incubator-beam/pull/592 [BEAM-245] Add Cassandra IO Be sure to do all of the following to help us incorporate your contribution quickly and easily: - [X] Make sure the PR title is formatted like: `[BEAM-] Description of pull request` - [X] Make sure tests pass via `mvn clean verify`. (Even better, enable Travis-CI on your fork and ensure the whole test matrix passes). - [X] Replace `` in the title with the actual Jira issue number, if there is one. - [X] If this contribution is large, please file an Apache [Individual Contributor License Agreement](https://www.apache.org/licenses/icla.txt). --- Initial version of CassandraIO. TODO: - fix and enable the tests (Cassandra daemon related) - usage of entity should be optional and the source should be able to return a `PCollection` You can merge this pull request into a Git repository by running: $ git pull https://github.com/jbonofre/incubator-beam BEAM-245-CASSANDRA Alternatively you can review and apply these changes as the patch at: https://github.com/apache/incubator-beam/pull/592.patch To close this pull request, make a commit to your master/trunk branch with (at least) the following in the commit message: This closes #592 commit 979c9f531aad0933fb7d1e3848897e50fc1ee64c Author: Jean-Baptiste Onofré <jbono...@apache.org> Date: 2016-07-02T08:36:37Z [BEAM-245] Add Cassandra IO --- If your project is set up for it, you can reply to this email and have your reply appear on GitHub as well. If your project does not have this feature enabled and wishes so, or if the feature is enabled but not working, please contact infrastructure at infrastruct...@apache.org or file a JIRA ticket with INFRA. ---
[GitHub] incubator-beam pull request #586: [BEAM-338] Cleanup Spark runner test resou...
GitHub user jbonofre opened a pull request: https://github.com/apache/incubator-beam/pull/586 [BEAM-338] Cleanup Spark runner test resources to avoid notice update Be sure to do all of the following to help us incorporate your contribution quickly and easily: - [X] Make sure the PR title is formatted like: `[BEAM-] Description of pull request` - [X] Make sure tests pass via `mvn clean verify`. (Even better, enable Travis-CI on your fork and ensure the whole test matrix passes). - [X] Replace `` in the title with the actual Jira issue number, if there is one. - [X] If this contribution is large, please file an Apache [Individual Contributor License Agreement](https://www.apache.org/licenses/icla.txt). --- Cleanup Spark runner test resources to avoid NOTICE & LICENSE update. You can merge this pull request into a Git repository by running: $ git pull https://github.com/jbonofre/incubator-beam BEAM-338 Alternatively you can review and apply these changes as the patch at: https://github.com/apache/incubator-beam/pull/586.patch To close this pull request, make a commit to your master/trunk branch with (at least) the following in the commit message: This closes #586 commit c81332e264c827a35c8c16a4f227e7c4837d1085 Author: Jean-Baptiste Onofré <jbono...@apache.org> Date: 2016-07-03T20:20:27Z [BEAM-338] Cleanup Spark runner test resources to avoid notice update --- If your project is set up for it, you can reply to this email and have your reply appear on GitHub as well. If your project does not have this feature enabled and wishes so, or if the feature is enabled but not working, please contact infrastructure at infrastruct...@apache.org or file a JIRA ticket with INFRA. ---
[1/2] incubator-beam git commit: [BEAM-390] Update spark dependency to the most recent stable version
Repository: incubator-beam Updated Branches: refs/heads/master 38866cd55 -> 61b9d723d [BEAM-390] Update spark dependency to the most recent stable version Project: http://git-wip-us.apache.org/repos/asf/incubator-beam/repo Commit: http://git-wip-us.apache.org/repos/asf/incubator-beam/commit/9b96a494 Tree: http://git-wip-us.apache.org/repos/asf/incubator-beam/tree/9b96a494 Diff: http://git-wip-us.apache.org/repos/asf/incubator-beam/diff/9b96a494 Branch: refs/heads/master Commit: 9b96a494576b0147a85431cbc0ad3c28dc228a7e Parents: 38866cd Author: Ismaël MejÃaAuthored: Wed Jun 29 23:01:09 2016 +0200 Committer: Jean-Baptiste Onofré Committed: Fri Jul 1 00:36:08 2016 +0200 -- runners/spark/pom.xml | 2 +- 1 file changed, 1 insertion(+), 1 deletion(-) -- http://git-wip-us.apache.org/repos/asf/incubator-beam/blob/9b96a494/runners/spark/pom.xml -- diff --git a/runners/spark/pom.xml b/runners/spark/pom.xml index 94c42bd..2a38923 100644 --- a/runners/spark/pom.xml +++ b/runners/spark/pom.xml @@ -34,7 +34,7 @@ UTF-8 UTF-8 -1.6.1 +1.6.2 2.2.0 0.8.2.1
[GitHub] incubator-beam pull request #420: [BEAM-287] Use flat groupId structure and ...
Github user jbonofre closed the pull request at: https://github.com/apache/incubator-beam/pull/420 --- If your project is set up for it, you can reply to this email and have your reply appear on GitHub as well. If your project does not have this feature enabled and wishes so, or if the feature is enabled but not working, please contact infrastructure at infrastruct...@apache.org or file a JIRA ticket with INFRA. ---
[GitHub] incubator-beam pull request #428: Fix src distribution content using the cor...
GitHub user jbonofre opened a pull request: https://github.com/apache/incubator-beam/pull/428 Fix src distribution content using the correct basedir location Be sure to do all of the following to help us incorporate your contribution quickly and easily: - [X] Make sure the PR title is formatted like: `[BEAM-] Description of pull request` - [X] Make sure tests pass via `mvn clean verify`. (Even better, enable Travis-CI on your fork and ensure the whole test matrix passes). - [X] Replace `` in the title with the actual Jira issue number, if there is one. - [X] If this contribution is large, please file an Apache [Individual Contributor License Agreement](https://www.apache.org/licenses/icla.txt). --- You can merge this pull request into a Git repository by running: $ git pull https://github.com/jbonofre/incubator-beam SRC_DISTRIBUTION Alternatively you can review and apply these changes as the patch at: https://github.com/apache/incubator-beam/pull/428.patch To close this pull request, make a commit to your master/trunk branch with (at least) the following in the commit message: This closes #428 commit 4e2c94c0a4dd9a82033df84ed49d798c77e3dc5a Author: Jean-Baptiste Onofré <jbono...@apache.org> Date: 2016-06-07T19:28:04Z Fix src distribution content using the correct basedir location --- If your project is set up for it, you can reply to this email and have your reply appear on GitHub as well. If your project does not have this feature enabled and wishes so, or if the feature is enabled but not working, please contact infrastructure at infrastruct...@apache.org or file a JIRA ticket with INFRA. ---