You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@beam.apache.org by am...@apache.org on 2016/04/23 00:23:13 UTC

[1/5] incubator-beam git commit: Remove specific registrar classes and service files

Repository: incubator-beam
Updated Branches:
  refs/heads/master cf4c3e204 -> 692f3a136


Remove specific registrar classes and service files

Add dependency on AutoService


Project: http://git-wip-us.apache.org/repos/asf/incubator-beam/repo
Commit: http://git-wip-us.apache.org/repos/asf/incubator-beam/commit/8a6ff228
Tree: http://git-wip-us.apache.org/repos/asf/incubator-beam/tree/8a6ff228
Diff: http://git-wip-us.apache.org/repos/asf/incubator-beam/diff/8a6ff228

Branch: refs/heads/master
Commit: 8a6ff2280ecc9b6df35941a5e74dfd04765ec6b2
Parents: cf4c3e2
Author: Sela <an...@paypal.com>
Authored: Fri Apr 22 14:33:46 2016 +0300
Committer: Sela <an...@paypal.com>
Committed: Sat Apr 23 01:10:51 2016 +0300

----------------------------------------------------------------------
 runners/spark/pom.xml                           |  6 ++++
 .../SparkPipelineOptionsRegistrar.java          | 31 -------------------
 .../SparkPipelineRunnerRegistrar.java           | 31 -------------------
 .../SparkStreamingPipelineOptionsRegistrar.java | 32 --------------------
 ...ataflow.sdk.options.PipelineOptionsRegistrar | 17 -----------
 ...dataflow.sdk.runners.PipelineRunnerRegistrar | 16 ----------
 6 files changed, 6 insertions(+), 127 deletions(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/incubator-beam/blob/8a6ff228/runners/spark/pom.xml
----------------------------------------------------------------------
diff --git a/runners/spark/pom.xml b/runners/spark/pom.xml
index f12d8a6..5ccaec5 100644
--- a/runners/spark/pom.xml
+++ b/runners/spark/pom.xml
@@ -71,6 +71,12 @@
       <version>${guava.version}</version>
     </dependency>
     <dependency>
+      <groupId>com.google.auto.service</groupId>
+      <artifactId>auto-service</artifactId>
+      <version>1.0-rc2</version>
+      <optional>true</optional>
+    </dependency>
+    <dependency>
       <groupId>org.apache.beam</groupId>
       <artifactId>java-sdk-all</artifactId>
       <version>${beam.version}</version>

http://git-wip-us.apache.org/repos/asf/incubator-beam/blob/8a6ff228/runners/spark/src/main/java/org/apache/beam/runners/spark/translation/SparkPipelineOptionsRegistrar.java
----------------------------------------------------------------------
diff --git a/runners/spark/src/main/java/org/apache/beam/runners/spark/translation/SparkPipelineOptionsRegistrar.java b/runners/spark/src/main/java/org/apache/beam/runners/spark/translation/SparkPipelineOptionsRegistrar.java
deleted file mode 100644
index c882d7b..0000000
--- a/runners/spark/src/main/java/org/apache/beam/runners/spark/translation/SparkPipelineOptionsRegistrar.java
+++ /dev/null
@@ -1,31 +0,0 @@
-/*
- * 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.runners.spark.translation;
-
-import com.google.common.collect.ImmutableList;
-import org.apache.beam.runners.spark.SparkPipelineOptions;
-import org.apache.beam.sdk.options.PipelineOptions;
-import org.apache.beam.sdk.options.PipelineOptionsRegistrar;
-
-public class SparkPipelineOptionsRegistrar implements PipelineOptionsRegistrar {
-  @Override
-  public Iterable<Class<? extends PipelineOptions>> getPipelineOptions() {
-    return ImmutableList.<Class<? extends PipelineOptions>>of(SparkPipelineOptions.class);
-  }
-}

http://git-wip-us.apache.org/repos/asf/incubator-beam/blob/8a6ff228/runners/spark/src/main/java/org/apache/beam/runners/spark/translation/SparkPipelineRunnerRegistrar.java
----------------------------------------------------------------------
diff --git a/runners/spark/src/main/java/org/apache/beam/runners/spark/translation/SparkPipelineRunnerRegistrar.java b/runners/spark/src/main/java/org/apache/beam/runners/spark/translation/SparkPipelineRunnerRegistrar.java
deleted file mode 100644
index 38993fb..0000000
--- a/runners/spark/src/main/java/org/apache/beam/runners/spark/translation/SparkPipelineRunnerRegistrar.java
+++ /dev/null
@@ -1,31 +0,0 @@
-/*
- * 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.runners.spark.translation;
-
-import com.google.common.collect.ImmutableList;
-import org.apache.beam.runners.spark.SparkPipelineRunner;
-import org.apache.beam.sdk.runners.PipelineRunner;
-import org.apache.beam.sdk.runners.PipelineRunnerRegistrar;
-
-public class SparkPipelineRunnerRegistrar implements PipelineRunnerRegistrar {
-  @Override
-  public Iterable<Class<? extends PipelineRunner<?>>> getPipelineRunners() {
-    return ImmutableList.<Class<? extends PipelineRunner<?>>>of(SparkPipelineRunner.class);
-  }
-}

http://git-wip-us.apache.org/repos/asf/incubator-beam/blob/8a6ff228/runners/spark/src/main/java/org/apache/beam/runners/spark/translation/streaming/SparkStreamingPipelineOptionsRegistrar.java
----------------------------------------------------------------------
diff --git a/runners/spark/src/main/java/org/apache/beam/runners/spark/translation/streaming/SparkStreamingPipelineOptionsRegistrar.java b/runners/spark/src/main/java/org/apache/beam/runners/spark/translation/streaming/SparkStreamingPipelineOptionsRegistrar.java
deleted file mode 100644
index 2e35098..0000000
--- a/runners/spark/src/main/java/org/apache/beam/runners/spark/translation/streaming/SparkStreamingPipelineOptionsRegistrar.java
+++ /dev/null
@@ -1,32 +0,0 @@
-/*
- * 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.runners.spark.translation.streaming;
-
-import com.google.common.collect.ImmutableList;
-import org.apache.beam.runners.spark.SparkStreamingPipelineOptions;
-import org.apache.beam.sdk.options.PipelineOptions;
-import org.apache.beam.sdk.options.PipelineOptionsRegistrar;
-
-public class SparkStreamingPipelineOptionsRegistrar implements PipelineOptionsRegistrar {
-
-  @Override
-  public Iterable<Class<? extends PipelineOptions>> getPipelineOptions() {
-    return ImmutableList.<Class<? extends PipelineOptions>>of(SparkStreamingPipelineOptions
-            .class);
-  }
-}

http://git-wip-us.apache.org/repos/asf/incubator-beam/blob/8a6ff228/runners/spark/src/main/resources/META-INF/services/com.google.cloud.dataflow.sdk.options.PipelineOptionsRegistrar
----------------------------------------------------------------------
diff --git a/runners/spark/src/main/resources/META-INF/services/com.google.cloud.dataflow.sdk.options.PipelineOptionsRegistrar b/runners/spark/src/main/resources/META-INF/services/com.google.cloud.dataflow.sdk.options.PipelineOptionsRegistrar
deleted file mode 100644
index e4a3a73..0000000
--- a/runners/spark/src/main/resources/META-INF/services/com.google.cloud.dataflow.sdk.options.PipelineOptionsRegistrar
+++ /dev/null
@@ -1,17 +0,0 @@
-#
-# Copyright 2014 Cloudera Inc.
-#
-# Licensed 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.
-#
-org.apache.beam.runners.spark.translation.SparkPipelineOptionsRegistrar
-org.apache.beam.runners.spark.translation.streaming.SparkStreamingPipelineOptionsRegistrar
\ No newline at end of file

http://git-wip-us.apache.org/repos/asf/incubator-beam/blob/8a6ff228/runners/spark/src/main/resources/META-INF/services/com.google.cloud.dataflow.sdk.runners.PipelineRunnerRegistrar
----------------------------------------------------------------------
diff --git a/runners/spark/src/main/resources/META-INF/services/com.google.cloud.dataflow.sdk.runners.PipelineRunnerRegistrar b/runners/spark/src/main/resources/META-INF/services/com.google.cloud.dataflow.sdk.runners.PipelineRunnerRegistrar
deleted file mode 100644
index 7949db4..0000000
--- a/runners/spark/src/main/resources/META-INF/services/com.google.cloud.dataflow.sdk.runners.PipelineRunnerRegistrar
+++ /dev/null
@@ -1,16 +0,0 @@
-#
-# Copyright 2014 Cloudera Inc.
-#
-# Licensed 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.
-#
-org.apache.beam.runners.spark.translation.SparkPipelineRunnerRegistrar
\ No newline at end of file


[3/5] incubator-beam git commit: Add SparkRunnerRegistrar as a runner specific registrar that uses AutoService

Posted by am...@apache.org.
Add SparkRunnerRegistrar as a runner specific registrar that uses AutoService


Project: http://git-wip-us.apache.org/repos/asf/incubator-beam/repo
Commit: http://git-wip-us.apache.org/repos/asf/incubator-beam/commit/88400476
Tree: http://git-wip-us.apache.org/repos/asf/incubator-beam/tree/88400476
Diff: http://git-wip-us.apache.org/repos/asf/incubator-beam/diff/88400476

Branch: refs/heads/master
Commit: 8840047657f0f629e3e21b1a000854fc4e2a4d5a
Parents: 8a6ff22
Author: Sela <an...@paypal.com>
Authored: Fri Apr 22 14:35:19 2016 +0300
Committer: Sela <an...@paypal.com>
Committed: Sat Apr 23 01:11:24 2016 +0300

----------------------------------------------------------------------
 .../runners/spark/SparkRunnerRegistrar.java     | 61 ++++++++++++++++++++
 1 file changed, 61 insertions(+)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/incubator-beam/blob/88400476/runners/spark/src/main/java/org/apache/beam/runners/spark/SparkRunnerRegistrar.java
----------------------------------------------------------------------
diff --git a/runners/spark/src/main/java/org/apache/beam/runners/spark/SparkRunnerRegistrar.java b/runners/spark/src/main/java/org/apache/beam/runners/spark/SparkRunnerRegistrar.java
new file mode 100644
index 0000000..30142f9
--- /dev/null
+++ b/runners/spark/src/main/java/org/apache/beam/runners/spark/SparkRunnerRegistrar.java
@@ -0,0 +1,61 @@
+/*
+ * 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.runners.spark;
+
+import com.google.auto.service.AutoService;
+import com.google.common.collect.ImmutableList;
+import org.apache.beam.sdk.options.PipelineOptions;
+import org.apache.beam.sdk.options.PipelineOptionsRegistrar;
+import org.apache.beam.sdk.runners.PipelineRunner;
+import org.apache.beam.sdk.runners.PipelineRunnerRegistrar;
+
+/**
+ * Contains the {@link PipelineRunnerRegistrar} and {@link PipelineOptionsRegistrar} for the
+ * {@link SparkPipelineRunner}.
+ *
+ * {@link AutoService} will register Spark's implementations of the {@link PipelineRunner}
+ * and {@link PipelineOptions} as available pipeline runner services.
+ */
+public final class SparkRunnerRegistrar {
+  private SparkRunnerRegistrar() {}
+
+  /**
+   * Registers the {@link SparkPipelineRunner}.
+   */
+  @AutoService(PipelineRunnerRegistrar.class)
+  public static class Runner implements PipelineRunnerRegistrar {
+    @Override
+    public Iterable<Class<? extends PipelineRunner<?>>> getPipelineRunners() {
+      return ImmutableList.<Class<? extends PipelineRunner<?>>>of(SparkPipelineRunner.class);
+    }
+  }
+
+  /**
+   * Registers the {@link SparkPipelineOptions} and {@link SparkStreamingPipelineOptions}.
+   */
+  @AutoService(PipelineOptionsRegistrar.class)
+  public static class Options implements PipelineOptionsRegistrar {
+    @Override
+    public Iterable<Class<? extends PipelineOptions>> getPipelineOptions() {
+      return ImmutableList.<Class<? extends PipelineOptions>>of(
+          SparkPipelineOptions.class,
+          SparkStreamingPipelineOptions.class);
+    }
+  }
+}


[4/5] incubator-beam git commit: Update README according to dataflow->beam package rename

Posted by am...@apache.org.
Update README according to dataflow->beam package rename

Annotate class with RunWith and some whitespace fix ups


Project: http://git-wip-us.apache.org/repos/asf/incubator-beam/repo
Commit: http://git-wip-us.apache.org/repos/asf/incubator-beam/commit/7fd9e1e7
Tree: http://git-wip-us.apache.org/repos/asf/incubator-beam/tree/7fd9e1e7
Diff: http://git-wip-us.apache.org/repos/asf/incubator-beam/diff/7fd9e1e7

Branch: refs/heads/master
Commit: 7fd9e1e70ef607c47613022c4b7cf63c46353e37
Parents: f424b8d
Author: Sela <an...@paypal.com>
Authored: Thu Apr 21 22:48:14 2016 +0300
Committer: Sela <an...@paypal.com>
Committed: Sat Apr 23 01:11:25 2016 +0300

----------------------------------------------------------------------
 runners/spark/README.md                                  |  6 +++---
 .../beam/runners/spark/SparkRunnerRegistrarTest.java     | 11 +++++++----
 2 files changed, 10 insertions(+), 7 deletions(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/incubator-beam/blob/7fd9e1e7/runners/spark/README.md
----------------------------------------------------------------------
diff --git a/runners/spark/README.md b/runners/spark/README.md
index 1d75b35..5b2e732 100644
--- a/runners/spark/README.md
+++ b/runners/spark/README.md
@@ -93,7 +93,7 @@ Switch to the Spark runner directory:
 Then run the [word count example][wc] from the SDK using a single threaded Spark instance
 in local mode:
 
-    mvn exec:exec -DmainClass=com.google.cloud.dataflow.examples.WordCount \
+    mvn exec:exec -DmainClass=org.apache.beam.examples.WordCount \
       -Dinput=/tmp/kinglear.txt -Doutput=/tmp/out -Drunner=SparkPipelineRunner \
       -DsparkMaster=local
 
@@ -104,7 +104,7 @@ Check the output by running:
 __Note: running examples using `mvn exec:exec` only works for Spark local mode at the
 moment. See the next section for how to run on a cluster.__
 
-[wc]: https://github.com/apache/incubator-beam/blob/master/examples/src/main/java/com/google/cloud/dataflow/examples/WordCount.java
+[wc]: https://github.com/apache/incubator-beam/blob/master/examples/java/src/main/java/org/apache/beam/examples/WordCount.java
 ## Running on a Cluster
 
 Spark Beam pipelines can be run on a cluster using the `spark-submit` command.
@@ -117,7 +117,7 @@ Then run the word count example using Spark submit with the `yarn-client` master
 (`yarn-cluster` works just as well):
 
     spark-submit \
-      --class com.google.cloud.dataflow.examples.WordCount \
+      --class org.apache.beam.examples.WordCount \
       --master yarn-client \
       target/spark-runner-*-spark-app.jar \
         --inputFile=kinglear.txt --output=out --runner=SparkPipelineRunner --sparkMaster=yarn-client

http://git-wip-us.apache.org/repos/asf/incubator-beam/blob/7fd9e1e7/runners/spark/src/test/java/org/apache/beam/runners/spark/SparkRunnerRegistrarTest.java
----------------------------------------------------------------------
diff --git a/runners/spark/src/test/java/org/apache/beam/runners/spark/SparkRunnerRegistrarTest.java b/runners/spark/src/test/java/org/apache/beam/runners/spark/SparkRunnerRegistrarTest.java
index d51403f..3643bac 100644
--- a/runners/spark/src/test/java/org/apache/beam/runners/spark/SparkRunnerRegistrarTest.java
+++ b/runners/spark/src/test/java/org/apache/beam/runners/spark/SparkRunnerRegistrarTest.java
@@ -23,6 +23,8 @@ import com.google.common.collect.Lists;
 import org.apache.beam.sdk.options.PipelineOptionsRegistrar;
 import org.apache.beam.sdk.runners.PipelineRunnerRegistrar;
 import org.junit.Test;
+import org.junit.runner.RunWith;
+import org.junit.runners.JUnit4;
 
 import java.util.ServiceLoader;
 
@@ -32,12 +34,13 @@ import static org.junit.Assert.fail;
 /**
  * Test {@link SparkRunnerRegistrar}.
  */
+@RunWith(JUnit4.class)
 public class SparkRunnerRegistrarTest {
   @Test
   public void testOptions() {
     assertEquals(
-	    ImmutableList.of(SparkPipelineOptions.class, SparkStreamingPipelineOptions.class),
-    	 new SparkRunnerRegistrar.Options().getPipelineOptions());
+        ImmutableList.of(SparkPipelineOptions.class, SparkStreamingPipelineOptions.class),
+        new SparkRunnerRegistrar.Options().getPipelineOptions());
   }
 
   @Test
@@ -51,7 +54,7 @@ public class SparkRunnerRegistrarTest {
     for (PipelineOptionsRegistrar registrar :
         Lists.newArrayList(ServiceLoader.load(PipelineOptionsRegistrar.class).iterator())) {
       if (registrar instanceof SparkRunnerRegistrar.Options) {
-      	return;
+        return;
       }
     }
     fail("Expected to find " + SparkRunnerRegistrar.Options.class);
@@ -62,7 +65,7 @@ public class SparkRunnerRegistrarTest {
     for (PipelineRunnerRegistrar registrar :
         Lists.newArrayList(ServiceLoader.load(PipelineRunnerRegistrar.class).iterator())) {
       if (registrar instanceof SparkRunnerRegistrar.Runner) {
-      	return;
+        return;
       }
     }
     fail("Expected to find " + SparkRunnerRegistrar.Runner.class);


[2/5] incubator-beam git commit: Add a unit test for the SparkRunnerRegistrar

Posted by am...@apache.org.
Add a unit test for the SparkRunnerRegistrar


Project: http://git-wip-us.apache.org/repos/asf/incubator-beam/repo
Commit: http://git-wip-us.apache.org/repos/asf/incubator-beam/commit/f424b8d8
Tree: http://git-wip-us.apache.org/repos/asf/incubator-beam/tree/f424b8d8
Diff: http://git-wip-us.apache.org/repos/asf/incubator-beam/diff/f424b8d8

Branch: refs/heads/master
Commit: f424b8d87aa491a29cf3c5cc3c162ee3e38aaf01
Parents: 8840047
Author: Sela <an...@paypal.com>
Authored: Fri Apr 22 14:36:33 2016 +0300
Committer: Sela <an...@paypal.com>
Committed: Sat Apr 23 01:11:24 2016 +0300

----------------------------------------------------------------------
 .../runners/spark/SparkRunnerRegistrarTest.java | 70 ++++++++++++++++++++
 1 file changed, 70 insertions(+)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/incubator-beam/blob/f424b8d8/runners/spark/src/test/java/org/apache/beam/runners/spark/SparkRunnerRegistrarTest.java
----------------------------------------------------------------------
diff --git a/runners/spark/src/test/java/org/apache/beam/runners/spark/SparkRunnerRegistrarTest.java b/runners/spark/src/test/java/org/apache/beam/runners/spark/SparkRunnerRegistrarTest.java
new file mode 100644
index 0000000..d51403f
--- /dev/null
+++ b/runners/spark/src/test/java/org/apache/beam/runners/spark/SparkRunnerRegistrarTest.java
@@ -0,0 +1,70 @@
+/*
+ * 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.runners.spark;
+
+import com.google.common.collect.ImmutableList;
+import com.google.common.collect.Lists;
+import org.apache.beam.sdk.options.PipelineOptionsRegistrar;
+import org.apache.beam.sdk.runners.PipelineRunnerRegistrar;
+import org.junit.Test;
+
+import java.util.ServiceLoader;
+
+import static org.junit.Assert.assertEquals;
+import static org.junit.Assert.fail;
+
+/**
+ * Test {@link SparkRunnerRegistrar}.
+ */
+public class SparkRunnerRegistrarTest {
+  @Test
+  public void testOptions() {
+    assertEquals(
+	    ImmutableList.of(SparkPipelineOptions.class, SparkStreamingPipelineOptions.class),
+    	 new SparkRunnerRegistrar.Options().getPipelineOptions());
+  }
+
+  @Test
+  public void testRunners() {
+    assertEquals(ImmutableList.of(SparkPipelineRunner.class),
+        new SparkRunnerRegistrar.Runner().getPipelineRunners());
+  }
+
+  @Test
+  public void testServiceLoaderForOptions() {
+    for (PipelineOptionsRegistrar registrar :
+        Lists.newArrayList(ServiceLoader.load(PipelineOptionsRegistrar.class).iterator())) {
+      if (registrar instanceof SparkRunnerRegistrar.Options) {
+      	return;
+      }
+    }
+    fail("Expected to find " + SparkRunnerRegistrar.Options.class);
+  }
+
+  @Test
+  public void testServiceLoaderForRunner() {
+    for (PipelineRunnerRegistrar registrar :
+        Lists.newArrayList(ServiceLoader.load(PipelineRunnerRegistrar.class).iterator())) {
+      if (registrar instanceof SparkRunnerRegistrar.Runner) {
+      	return;
+      }
+    }
+    fail("Expected to find " + SparkRunnerRegistrar.Runner.class);
+  }
+}


[5/5] incubator-beam git commit: [BEAM-213] Fix README to use refactored package names and use AutoService for Registrar

Posted by am...@apache.org.
[BEAM-213] Fix README to use refactored package names and use AutoService for Registrar

This closes #230


Project: http://git-wip-us.apache.org/repos/asf/incubator-beam/repo
Commit: http://git-wip-us.apache.org/repos/asf/incubator-beam/commit/692f3a13
Tree: http://git-wip-us.apache.org/repos/asf/incubator-beam/tree/692f3a13
Diff: http://git-wip-us.apache.org/repos/asf/incubator-beam/diff/692f3a13

Branch: refs/heads/master
Commit: 692f3a136d00469d0b5bdc8aecec2db5ab4ac357
Parents: cf4c3e2 7fd9e1e
Author: Sela <an...@paypal.com>
Authored: Sat Apr 23 01:15:32 2016 +0300
Committer: Sela <an...@paypal.com>
Committed: Sat Apr 23 01:15:32 2016 +0300

----------------------------------------------------------------------
 runners/spark/README.md                         |  6 +-
 runners/spark/pom.xml                           |  6 ++
 .../runners/spark/SparkRunnerRegistrar.java     | 61 ++++++++++++++++
 .../SparkPipelineOptionsRegistrar.java          | 31 ---------
 .../SparkPipelineRunnerRegistrar.java           | 31 ---------
 .../SparkStreamingPipelineOptionsRegistrar.java | 32 ---------
 ...ataflow.sdk.options.PipelineOptionsRegistrar | 17 -----
 ...dataflow.sdk.runners.PipelineRunnerRegistrar | 16 -----
 .../runners/spark/SparkRunnerRegistrarTest.java | 73 ++++++++++++++++++++
 9 files changed, 143 insertions(+), 130 deletions(-)
----------------------------------------------------------------------