Skip to content

Commit

Permalink
This closes #176
Browse files Browse the repository at this point in the history
  • Loading branch information
davorbonaci committed Apr 14, 2016
2 parents c4cbbb1 + 75a1905 commit 96765f1
Show file tree
Hide file tree
Showing 979 changed files with 7,699 additions and 7,225 deletions.
8 changes: 4 additions & 4 deletions contrib/hadoop/pom.xml
Original file line number Diff line number Diff line change
Expand Up @@ -37,7 +37,7 @@

<properties>
<project.build.sourceEncoding>UTF-8</project.build.sourceEncoding>
<google-cloud-dataflow-version>[1.2.0,2.0.0)</google-cloud-dataflow-version>
<beam-version>[0.1.0, 1.0.0)</beam-version>
</properties>

<build>
Expand Down Expand Up @@ -137,9 +137,9 @@

<dependencies>
<dependency>
<groupId>com.google.cloud.dataflow</groupId>
<artifactId>google-cloud-dataflow-java-sdk-all</artifactId>
<version>${google-cloud-dataflow-version}</version>
<groupId>org.apache.beam</groupId>
<artifactId>java-sdk-all</artifactId>
<version>${beam-version}</version>
</dependency>

<!-- @tomwhite: Hadoop doesn't have great RPC client compatibility between one version and
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -15,15 +15,15 @@
* See the License for the specific language governing permissions and
* limitations under the License.
*/
package com.google.cloud.dataflow.contrib.hadoop;

import com.google.cloud.dataflow.sdk.coders.Coder;
import com.google.cloud.dataflow.sdk.coders.KvCoder;
import com.google.cloud.dataflow.sdk.coders.VoidCoder;
import com.google.cloud.dataflow.sdk.io.BoundedSource;
import com.google.cloud.dataflow.sdk.io.Read;
import com.google.cloud.dataflow.sdk.options.PipelineOptions;
import com.google.cloud.dataflow.sdk.values.KV;
package org.apache.beam.contrib.hadoop;

import org.apache.beam.sdk.coders.Coder;
import org.apache.beam.sdk.coders.KvCoder;
import org.apache.beam.sdk.coders.VoidCoder;
import org.apache.beam.sdk.io.BoundedSource;
import org.apache.beam.sdk.io.Read;
import org.apache.beam.sdk.options.PipelineOptions;
import org.apache.beam.sdk.values.KV;
import com.google.common.base.Function;
import com.google.common.base.Preconditions;
import com.google.common.collect.ImmutableList;
Expand Down Expand Up @@ -56,14 +56,14 @@
* A {@code BoundedSource} for reading files resident in a Hadoop filesystem using a
* Hadoop file-based input format.
*
* <p>To read a {@link com.google.cloud.dataflow.sdk.values.PCollection} of
* {@link com.google.cloud.dataflow.sdk.values.KV} key-value pairs from one or more
* <p>To read a {@link org.apache.beam.sdk.values.PCollection} of
* {@link org.apache.beam.sdk.values.KV} key-value pairs from one or more
* Hadoop files, use {@link HadoopFileSource#from} to specify the path(s) of the files to
* read, the Hadoop {@link org.apache.hadoop.mapreduce.lib.input.FileInputFormat}, the
* key class and the value class.
*
* <p>A {@code HadoopFileSource} can be read from using the
* {@link com.google.cloud.dataflow.sdk.io.Read} transform. For example:
* {@link org.apache.beam.sdk.io.Read} transform. For example:
*
* <pre>
* {@code
Expand All @@ -85,7 +85,7 @@
*
* Implementation note: Since Hadoop's {@link org.apache.hadoop.mapreduce.lib.input.FileInputFormat}
* determines the input splits, this class extends {@link BoundedSource} rather than
* {@link com.google.cloud.dataflow.sdk.io.OffsetBasedSource}, since the latter
* {@link org.apache.beam.sdk.io.OffsetBasedSource}, since the latter
* dictates input splits.
* @param <K> The type of keys to be read from the source.
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -15,12 +15,12 @@
* See the License for the specific language governing permissions and
* limitations under the License.
*/
package com.google.cloud.dataflow.contrib.hadoop;
package org.apache.beam.contrib.hadoop;

import com.google.cloud.dataflow.sdk.coders.Coder;
import com.google.cloud.dataflow.sdk.coders.CoderException;
import com.google.cloud.dataflow.sdk.coders.StandardCoder;
import com.google.cloud.dataflow.sdk.util.CloudObject;
import org.apache.beam.sdk.coders.Coder;
import org.apache.beam.sdk.coders.CoderException;
import org.apache.beam.sdk.coders.StandardCoder;
import org.apache.beam.sdk.util.CloudObject;
import com.fasterxml.jackson.annotation.JsonCreator;
import com.fasterxml.jackson.annotation.JsonProperty;
import org.apache.hadoop.io.Writable;
Expand All @@ -32,7 +32,7 @@
import java.util.List;

/**
* A {@code WritableCoder} is a {@link com.google.cloud.dataflow.sdk.coders.Coder} for a
* A {@code WritableCoder} is a {@link org.apache.beam.sdk.coders.Coder} for a
* Java class that implements {@link org.apache.hadoop.io.Writable}.
*
* <p> To use, specify the coder type on a PCollection:
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -15,21 +15,21 @@
* See the License for the specific language governing permissions and
* limitations under the License.
*/
package com.google.cloud.dataflow.contrib.hadoop;
package org.apache.beam.contrib.hadoop;

import static com.google.cloud.dataflow.sdk.testing.SourceTestUtils.readFromSource;
import static org.apache.beam.sdk.testing.SourceTestUtils.readFromSource;
import static org.hamcrest.Matchers.containsInAnyOrder;
import static org.junit.Assert.assertEquals;
import static org.junit.Assert.assertThat;
import static org.junit.Assert.assertTrue;
import static org.junit.Assert.fail;

import com.google.cloud.dataflow.sdk.io.BoundedSource;
import com.google.cloud.dataflow.sdk.io.Source;
import com.google.cloud.dataflow.sdk.options.PipelineOptions;
import com.google.cloud.dataflow.sdk.options.PipelineOptionsFactory;
import com.google.cloud.dataflow.sdk.testing.SourceTestUtils;
import com.google.cloud.dataflow.sdk.values.KV;
import org.apache.beam.sdk.io.BoundedSource;
import org.apache.beam.sdk.io.Source;
import org.apache.beam.sdk.options.PipelineOptions;
import org.apache.beam.sdk.options.PipelineOptionsFactory;
import org.apache.beam.sdk.testing.SourceTestUtils;
import org.apache.beam.sdk.values.KV;

import org.apache.hadoop.conf.Configuration;
import org.apache.hadoop.fs.Path;
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -15,9 +15,9 @@
* See the License for the specific language governing permissions and
* limitations under the License.
*/
package com.google.cloud.dataflow.contrib.hadoop;
package org.apache.beam.contrib.hadoop;

import com.google.cloud.dataflow.sdk.testing.CoderProperties;
import org.apache.beam.sdk.testing.CoderProperties;

import org.apache.hadoop.io.IntWritable;
import org.junit.Test;
Expand Down
8 changes: 4 additions & 4 deletions contrib/join-library/pom.xml
Original file line number Diff line number Diff line change
Expand Up @@ -50,7 +50,7 @@

<properties>
<project.build.sourceEncoding>UTF-8</project.build.sourceEncoding>
<google-cloud-dataflow-version>[1.0.0, 2.0.0)</google-cloud-dataflow-version>
<beam-version>[0.1.0, 1.0.0)</beam-version>
</properties>

<build>
Expand Down Expand Up @@ -155,9 +155,9 @@

<dependencies>
<dependency>
<groupId>com.google.cloud.dataflow</groupId>
<artifactId>google-cloud-dataflow-java-sdk-all</artifactId>
<version>${google-cloud-dataflow-version}</version>
<groupId>org.apache.beam</groupId>
<artifactId>java-sdk-all</artifactId>
<version>${beam-version}</version>
</dependency>

<dependency>
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -15,17 +15,17 @@
* See the License for the specific language governing permissions and
* limitations under the License.
*/
package com.google.cloud.dataflow.contrib.joinlibrary;

import com.google.cloud.dataflow.sdk.coders.KvCoder;
import com.google.cloud.dataflow.sdk.transforms.DoFn;
import com.google.cloud.dataflow.sdk.transforms.ParDo;
import com.google.cloud.dataflow.sdk.transforms.join.CoGbkResult;
import com.google.cloud.dataflow.sdk.transforms.join.CoGroupByKey;
import com.google.cloud.dataflow.sdk.transforms.join.KeyedPCollectionTuple;
import com.google.cloud.dataflow.sdk.values.KV;
import com.google.cloud.dataflow.sdk.values.PCollection;
import com.google.cloud.dataflow.sdk.values.TupleTag;
package org.apache.beam.contrib.joinlibrary;

import org.apache.beam.sdk.coders.KvCoder;
import org.apache.beam.sdk.transforms.DoFn;
import org.apache.beam.sdk.transforms.ParDo;
import org.apache.beam.sdk.transforms.join.CoGbkResult;
import org.apache.beam.sdk.transforms.join.CoGroupByKey;
import org.apache.beam.sdk.transforms.join.KeyedPCollectionTuple;
import org.apache.beam.sdk.values.KV;
import org.apache.beam.sdk.values.PCollection;
import org.apache.beam.sdk.values.TupleTag;
import com.google.common.base.Preconditions;

/**
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -15,14 +15,14 @@
* See the License for the specific language governing permissions and
* limitations under the License.
*/
package com.google.cloud.dataflow.contrib.joinlibrary;

import com.google.cloud.dataflow.sdk.Pipeline;
import com.google.cloud.dataflow.sdk.testing.PAssert;
import com.google.cloud.dataflow.sdk.testing.TestPipeline;
import com.google.cloud.dataflow.sdk.transforms.Create;
import com.google.cloud.dataflow.sdk.values.KV;
import com.google.cloud.dataflow.sdk.values.PCollection;
package org.apache.beam.contrib.joinlibrary;

import org.apache.beam.sdk.Pipeline;
import org.apache.beam.sdk.testing.PAssert;
import org.apache.beam.sdk.testing.TestPipeline;
import org.apache.beam.sdk.transforms.Create;
import org.apache.beam.sdk.values.KV;
import org.apache.beam.sdk.values.PCollection;

import org.junit.Before;
import org.junit.Test;
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -15,14 +15,14 @@
* See the License for the specific language governing permissions and
* limitations under the License.
*/
package com.google.cloud.dataflow.contrib.joinlibrary;

import com.google.cloud.dataflow.sdk.Pipeline;
import com.google.cloud.dataflow.sdk.testing.PAssert;
import com.google.cloud.dataflow.sdk.testing.TestPipeline;
import com.google.cloud.dataflow.sdk.transforms.Create;
import com.google.cloud.dataflow.sdk.values.KV;
import com.google.cloud.dataflow.sdk.values.PCollection;
package org.apache.beam.contrib.joinlibrary;

import org.apache.beam.sdk.Pipeline;
import org.apache.beam.sdk.testing.PAssert;
import org.apache.beam.sdk.testing.TestPipeline;
import org.apache.beam.sdk.transforms.Create;
import org.apache.beam.sdk.values.KV;
import org.apache.beam.sdk.values.PCollection;

import org.junit.Before;
import org.junit.Test;
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -15,14 +15,14 @@
* See the License for the specific language governing permissions and
* limitations under the License.
*/
package com.google.cloud.dataflow.contrib.joinlibrary;

import com.google.cloud.dataflow.sdk.Pipeline;
import com.google.cloud.dataflow.sdk.testing.PAssert;
import com.google.cloud.dataflow.sdk.testing.TestPipeline;
import com.google.cloud.dataflow.sdk.transforms.Create;
import com.google.cloud.dataflow.sdk.values.KV;
import com.google.cloud.dataflow.sdk.values.PCollection;
package org.apache.beam.contrib.joinlibrary;

import org.apache.beam.sdk.Pipeline;
import org.apache.beam.sdk.testing.PAssert;
import org.apache.beam.sdk.testing.TestPipeline;
import org.apache.beam.sdk.transforms.Create;
import org.apache.beam.sdk.values.KV;
import org.apache.beam.sdk.values.PCollection;

import org.junit.Before;
import org.junit.Test;
Expand Down
4 changes: 2 additions & 2 deletions examples/java/pom.xml
Original file line number Diff line number Diff line change
Expand Up @@ -104,8 +104,8 @@
<windowtitle>Apache Beam Examples</windowtitle>
<doctitle>Apache Beam Examples</doctitle>

<subpackages>com.google.cloud.dataflow.examples</subpackages>
<additionalparam>-exclude com.google.cloud.dataflow.sdk.runners.worker:com.google.cloud.dataflow.sdk.runners.dataflow:com.google.cloud.dataflow.sdk.util ${dataflow.javadoc_opts}</additionalparam>
<subpackages>org.apache.beam.examples</subpackages>
<additionalparam>-exclude org.apache.beam.sdk.runners.worker:org.apache.beam.sdk.runners.dataflow:org.apache.beam.sdk.util ${dataflow.javadoc_opts}</additionalparam>
<use>false</use>
<quiet>true</quiet>
<bottom><![CDATA[<br>]]></bottom>
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -15,20 +15,20 @@
* See the License for the specific language governing permissions and
* limitations under the License.
*/
package com.google.cloud.dataflow.examples;
package org.apache.beam.examples;

import com.google.cloud.dataflow.sdk.Pipeline;
import com.google.cloud.dataflow.sdk.io.TextIO;
import com.google.cloud.dataflow.sdk.options.Default;
import com.google.cloud.dataflow.sdk.options.Description;
import com.google.cloud.dataflow.sdk.options.PipelineOptionsFactory;
import com.google.cloud.dataflow.sdk.testing.PAssert;
import com.google.cloud.dataflow.sdk.transforms.Aggregator;
import com.google.cloud.dataflow.sdk.transforms.DoFn;
import com.google.cloud.dataflow.sdk.transforms.ParDo;
import com.google.cloud.dataflow.sdk.transforms.Sum;
import com.google.cloud.dataflow.sdk.values.KV;
import com.google.cloud.dataflow.sdk.values.PCollection;
import org.apache.beam.sdk.Pipeline;
import org.apache.beam.sdk.io.TextIO;
import org.apache.beam.sdk.options.Default;
import org.apache.beam.sdk.options.Description;
import org.apache.beam.sdk.options.PipelineOptionsFactory;
import org.apache.beam.sdk.testing.PAssert;
import org.apache.beam.sdk.transforms.Aggregator;
import org.apache.beam.sdk.transforms.DoFn;
import org.apache.beam.sdk.transforms.ParDo;
import org.apache.beam.sdk.transforms.Sum;
import org.apache.beam.sdk.values.KV;
import org.apache.beam.sdk.values.PCollection;

import org.slf4j.Logger;
import org.slf4j.LoggerFactory;
Expand Down Expand Up @@ -70,7 +70,7 @@
* --project=YOUR_PROJECT_ID
* --stagingLocation=gs://YOUR_STAGING_DIRECTORY
* --runner=BlockingDataflowPipelineRunner
* --workerLogLevelOverrides={"com.google.cloud.dataflow.examples":"DEBUG"}
* --workerLogLevelOverrides={"org.apache.beam.examples":"DEBUG"}
* }
* </pre>
*
Expand All @@ -79,7 +79,7 @@
* <pre>
* mvn compile exec:java ... \
* -Dexec.args="... \
* --workerLogLevelOverrides={\\\"com.google.cloud.dataflow.examples\\\":\\\"DEBUG\\\"}"
* --workerLogLevelOverrides={\\\"org.apache.beam.examples\\\":\\\"DEBUG\\\"}"
* </pre>
*
* <p>Concept #2: Dataflow workers which execute user code are configured to log to Cloud
Expand All @@ -90,10 +90,10 @@
* </code></pre>
* For example, by specifying:
* <pre><code>
* --workerLogLevelOverrides={"com.google.cloud.dataflow.examples":"DEBUG"}
* --workerLogLevelOverrides={"org.apache.beam.examples":"DEBUG"}
* </code></pre>
* when executing this pipeline using the Dataflow service, Cloud Logging would contain only
* "DEBUG" or higher level logs for the {@code com.google.cloud.dataflow.examples} package in
* "DEBUG" or higher level logs for the {@code org.apache.beam.examples} package in
* addition to the default "INFO" or higher level logs. In addition, the default Dataflow worker
* logging configuration can be overridden by specifying
* {@code --defaultWorkerLogLevel=<one of TRACE, DEBUG, INFO, WARN, ERROR>}. For example,
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -15,19 +15,19 @@
* See the License for the specific language governing permissions and
* limitations under the License.
*/
package com.google.cloud.dataflow.examples;
package org.apache.beam.examples;

import com.google.cloud.dataflow.sdk.Pipeline;
import com.google.cloud.dataflow.sdk.io.TextIO;
import com.google.cloud.dataflow.sdk.options.DataflowPipelineOptions;
import com.google.cloud.dataflow.sdk.options.PipelineOptionsFactory;
import com.google.cloud.dataflow.sdk.runners.BlockingDataflowPipelineRunner;
import com.google.cloud.dataflow.sdk.transforms.Count;
import com.google.cloud.dataflow.sdk.transforms.DoFn;
import com.google.cloud.dataflow.sdk.transforms.MapElements;
import com.google.cloud.dataflow.sdk.transforms.ParDo;
import com.google.cloud.dataflow.sdk.transforms.SimpleFunction;
import com.google.cloud.dataflow.sdk.values.KV;
import org.apache.beam.sdk.Pipeline;
import org.apache.beam.sdk.io.TextIO;
import org.apache.beam.sdk.options.DataflowPipelineOptions;
import org.apache.beam.sdk.options.PipelineOptionsFactory;
import org.apache.beam.sdk.runners.BlockingDataflowPipelineRunner;
import org.apache.beam.sdk.transforms.Count;
import org.apache.beam.sdk.transforms.DoFn;
import org.apache.beam.sdk.transforms.MapElements;
import org.apache.beam.sdk.transforms.ParDo;
import org.apache.beam.sdk.transforms.SimpleFunction;
import org.apache.beam.sdk.values.KV;


/**
Expand Down

0 comments on commit 96765f1

Please sign in to comment.