[
https://issues.apache.org/jira/browse/BEAM-4306?focusedWorklogId=106176&page=com.atlassian.jira.plugin.system.issuetabpanels:worklog-tabpanel#worklog-106176
]
ASF GitHub Bot logged work on BEAM-4306:
----------------------------------------
Author: ASF GitHub Bot
Created on: 26/May/18 18:23
Start Date: 26/May/18 18:23
Worklog Time Spent: 10m
Work Description: tweise closed pull request #5475: [BEAM-4306] Enforce
ErrorProne analysis in apex runner
URL: https://github.com/apache/beam/pull/5475
This is a PR merged from a forked repository.
As GitHub hides the original diff on merge, it is displayed below for
the sake of provenance:
As this is a foreign pull request (from a fork), the diff is supplied
below (as it won't show otherwise due to GitHub magic):
diff --git a/runners/apex/build.gradle b/runners/apex/build.gradle
index e6ffdc09631..4960de374ef 100644
--- a/runners/apex/build.gradle
+++ b/runners/apex/build.gradle
@@ -19,7 +19,7 @@
import groovy.json.JsonOutput
apply from: project(":").file("build_rules.gradle")
-applyJavaNature()
+applyJavaNature(failOnWarning: true)
description = "Apache Beam :: Runners :: Apex"
@@ -35,6 +35,7 @@ configurations {
}
dependencies {
+ compileOnly library.java.findbugs_annotations
shadow project(path: ":beam-model-pipeline", configuration: "shadow")
shadow project(path: ":beam-sdks-java-core", configuration: "shadow")
shadow project(path: ":beam-runners-core-construction-java", configuration:
"shadow")
@@ -45,6 +46,7 @@ dependencies {
shadow library.java.commons_lang3
shadow library.java.findbugs_jsr305
shadow library.java.apex_engine
+ testCompileOnly library.java.findbugs_annotations
shadowTest project(path: ":beam-sdks-java-core", configuration: "shadowTest")
// ApexStateInternalsTest extends abstract StateInternalsTest
shadowTest project(path: ":beam-runners-core-java", configuration:
"shadowTest")
diff --git
a/runners/apex/src/main/java/org/apache/beam/runners/apex/ApexYarnLauncher.java
b/runners/apex/src/main/java/org/apache/beam/runners/apex/ApexYarnLauncher.java
index 59d3c960888..3ac2c8c4857 100644
---
a/runners/apex/src/main/java/org/apache/beam/runners/apex/ApexYarnLauncher.java
+++
b/runners/apex/src/main/java/org/apache/beam/runners/apex/ApexYarnLauncher.java
@@ -24,6 +24,7 @@
import com.datatorrent.api.DAG;
import com.datatorrent.api.StreamingApplication;
import com.google.common.annotations.VisibleForTesting;
+import com.google.common.base.Splitter;
import com.google.common.collect.Sets;
import java.io.BufferedReader;
import java.io.ByteArrayOutputStream;
@@ -41,6 +42,7 @@
import java.net.URI;
import java.net.URL;
import java.net.URLClassLoader;
+import java.nio.charset.StandardCharsets;
import java.nio.file.FileSystem;
import java.nio.file.FileSystems;
import java.nio.file.FileVisitResult;
@@ -173,7 +175,8 @@ public void shutdown(ShutdownMode arg0) throws
LauncherException {
*/
public static List<File> getYarnDeployDependencies() throws IOException {
try (InputStream dependencyTree =
ApexRunner.class.getResourceAsStream("dependency-tree")) {
- try (BufferedReader br = new BufferedReader(new
InputStreamReader(dependencyTree))) {
+ try (BufferedReader br =
+ new BufferedReader(new InputStreamReader(dependencyTree,
StandardCharsets.UTF_8))) {
String line;
List<String> excludes = new ArrayList<>();
int excludeLevel = Integer.MAX_VALUE;
@@ -198,7 +201,8 @@ public void shutdown(ShutdownMode arg0) throws
LauncherException {
Set<String> excludeJarFileNames = Sets.newHashSet();
for (String exclude : excludes) {
- String[] mvnc = exclude.split(":");
+ List<String> strings = Splitter.on(':').splitToList(exclude);
+ String[] mvnc = strings.toArray(new String[strings.size()]);
String fileName = mvnc[1] + "-";
if (mvnc.length == 6) {
fileName += mvnc[4] + "-" + mvnc[3]; // with classifier
@@ -312,7 +316,7 @@ public static void main(String[] args) throws IOException {
checkArgument(args.length == 1, "exactly one argument expected");
File file = new File(args[0]);
checkArgument(file.exists() && file.isFile(), "invalid file path %s",
file);
- final LaunchParams params = (LaunchParams) SerializationUtils.deserialize(
+ final LaunchParams params = SerializationUtils.deserialize(
new FileInputStream(file));
StreamingApplication apexApp = (dag, conf) -> copyShallow(params.dag, dag);
Configuration conf = new Configuration(); // configuration from Hadoop
client
diff --git
a/runners/apex/src/main/java/org/apache/beam/runners/apex/TestApexRunner.java
b/runners/apex/src/main/java/org/apache/beam/runners/apex/TestApexRunner.java
index b68d3da5891..0f6d9e6fe65 100644
---
a/runners/apex/src/main/java/org/apache/beam/runners/apex/TestApexRunner.java
+++
b/runners/apex/src/main/java/org/apache/beam/runners/apex/TestApexRunner.java
@@ -25,9 +25,7 @@
import org.apache.beam.sdk.options.PipelineOptionsValidator;
import org.joda.time.Duration;
-/**
- * Apex {@link PipelineRunner} for testing.
- */
+/** Apex {@link PipelineRunner} for testing. */
public class TestApexRunner extends PipelineRunner<ApexRunnerResult> {
private static final int RUN_WAIT_MILLIS = 20000;
@@ -48,11 +46,11 @@ public static TestApexRunner fromOptions(PipelineOptions
options) {
public static DAG translate(Pipeline pipeline, ApexPipelineOptions options) {
ApexRunner delegate = new ApexRunner(options);
delegate.translateOnly = true;
- DAG dag = delegate.run(pipeline).getApexDAG();
- return dag;
+ return delegate.run(pipeline).getApexDAG();
}
@Override
+ @SuppressWarnings("Finally")
public ApexRunnerResult run(Pipeline pipeline) {
ApexRunnerResult result = delegate.run(pipeline);
try {
diff --git
a/runners/apex/src/main/java/org/apache/beam/runners/apex/translation/FlattenPCollectionTranslator.java
b/runners/apex/src/main/java/org/apache/beam/runners/apex/translation/FlattenPCollectionTranslator.java
index 189cb656380..28cda0d194e 100644
---
a/runners/apex/src/main/java/org/apache/beam/runners/apex/translation/FlattenPCollectionTranslator.java
+++
b/runners/apex/src/main/java/org/apache/beam/runners/apex/translation/FlattenPCollectionTranslator.java
@@ -27,7 +27,6 @@
import org.apache.beam.runners.apex.translation.operators.ApexFlattenOperator;
import
org.apache.beam.runners.apex.translation.operators.ApexReadUnboundedInputOperator;
import org.apache.beam.runners.apex.translation.utils.ValuesSource;
-import org.apache.beam.sdk.coders.Coder;
import org.apache.beam.sdk.coders.VoidCoder;
import org.apache.beam.sdk.io.UnboundedSource;
import org.apache.beam.sdk.transforms.Flatten;
@@ -50,7 +49,7 @@ public void translate(Flatten.PCollections<T> transform,
TranslationContext cont
// create a dummy source that never emits anything
@SuppressWarnings("unchecked")
UnboundedSource<T, ?> unboundedSource = new
ValuesSource<>(Collections.EMPTY_LIST,
- (Coder<T>) VoidCoder.of());
+ VoidCoder.of());
ApexReadUnboundedInputOperator<T, ?> operator = new
ApexReadUnboundedInputOperator<>(
unboundedSource, context.getPipelineOptions());
context.addOperator(operator, operator.output);
diff --git
a/runners/apex/src/main/java/org/apache/beam/runners/apex/translation/TranslationContext.java
b/runners/apex/src/main/java/org/apache/beam/runners/apex/translation/TranslationContext.java
index 94d13e177de..e252f2be847 100644
---
a/runners/apex/src/main/java/org/apache/beam/runners/apex/translation/TranslationContext.java
+++
b/runners/apex/src/main/java/org/apache/beam/runners/apex/translation/TranslationContext.java
@@ -52,7 +52,7 @@
/**
* Maintains context data for {@link TransformTranslator}s.
*/
-@SuppressWarnings({"rawtypes", "unchecked"})
+@SuppressWarnings({"rawtypes", "unchecked", "TypeParameterUnusedInFormals"})
class TranslationContext {
private final ApexPipelineOptions pipelineOptions;
@@ -112,7 +112,7 @@ public String getFullName() {
}
public void addOperator(Operator operator, OutputPort port) {
- addOperator(operator, port, (PCollection<?>) getOutput());
+ addOperator(operator, port, getOutput());
}
/**
diff --git
a/runners/apex/src/main/java/org/apache/beam/runners/apex/translation/operators/ApexTimerInternals.java
b/runners/apex/src/main/java/org/apache/beam/runners/apex/translation/operators/ApexTimerInternals.java
index 0a0dd5086d8..b3936b39021 100644
---
a/runners/apex/src/main/java/org/apache/beam/runners/apex/translation/operators/ApexTimerInternals.java
+++
b/runners/apex/src/main/java/org/apache/beam/runners/apex/translation/operators/ApexTimerInternals.java
@@ -45,7 +45,6 @@
* <p>Assumes that the current key is set prior to accessing the state.<br>
* This implementation stores timer data in heap memory and is serialized
* during checkpointing, it will only work with a small number of timers.
- * @param <K>
*/
@DefaultSerializer(JavaSerializer.class)
class ApexTimerInternals<K> implements TimerInternals, Serializable {
diff --git
a/runners/apex/src/main/java/org/apache/beam/runners/apex/translation/utils/ApexStateInternals.java
b/runners/apex/src/main/java/org/apache/beam/runners/apex/translation/utils/ApexStateInternals.java
index cc73d2896ca..092903ee3dc 100644
---
a/runners/apex/src/main/java/org/apache/beam/runners/apex/translation/utils/ApexStateInternals.java
+++
b/runners/apex/src/main/java/org/apache/beam/runners/apex/translation/utils/ApexStateInternals.java
@@ -295,7 +295,7 @@ public TimestampCombiner getTimestampCombiner() {
}
- private final class ApexCombiningState<K, InputT, AccumT, OutputT>
+ private final class ApexCombiningState<InputT, AccumT, OutputT>
extends AbstractState<AccumT>
implements CombiningState<InputT, AccumT, OutputT> {
private final CombineFn<InputT, AccumT, OutputT> combineFn;
@@ -309,7 +309,7 @@ private ApexCombiningState(StateNamespace namespace,
}
@Override
- public ApexCombiningState<K, InputT, AccumT, OutputT> readLater() {
+ public ApexCombiningState<InputT, AccumT, OutputT> readLater() {
return this;
}
diff --git
a/runners/apex/src/main/java/org/apache/beam/runners/apex/translation/utils/StateInternalsProxy.java
b/runners/apex/src/main/java/org/apache/beam/runners/apex/translation/utils/StateInternalsProxy.java
index b652c68795e..b0f19a238b6 100644
---
a/runners/apex/src/main/java/org/apache/beam/runners/apex/translation/utils/StateInternalsProxy.java
+++
b/runners/apex/src/main/java/org/apache/beam/runners/apex/translation/utils/StateInternalsProxy.java
@@ -28,10 +28,7 @@
import org.apache.beam.sdk.state.State;
import org.apache.beam.sdk.state.StateContext;
-/**
- * State internals for reusable processing context.
- * @param <K>
- */
+/** State internals for reusable processing context. */
@DefaultSerializer(JavaSerializer.class)
public class StateInternalsProxy<K> implements StateInternals, Serializable {
diff --git
a/runners/apex/src/test/java/org/apache/beam/runners/apex/ApexRunnerTest.java
b/runners/apex/src/test/java/org/apache/beam/runners/apex/ApexRunnerTest.java
index 2b1fbef7908..554811221b3 100644
---
a/runners/apex/src/test/java/org/apache/beam/runners/apex/ApexRunnerTest.java
+++
b/runners/apex/src/test/java/org/apache/beam/runners/apex/ApexRunnerTest.java
@@ -38,17 +38,13 @@
import org.junit.Assert;
import org.junit.Test;
-/**
- * Tests for the Apex runner.
- */
+/** Tests for the Apex runner. */
public class ApexRunnerTest {
@Test
public void testConfigProperties() throws Exception {
-
String operName = "testProperties";
- ApexPipelineOptions options = PipelineOptionsFactory.create()
- .as(ApexPipelineOptions.class);
+ ApexPipelineOptions options =
PipelineOptionsFactory.create().as(ApexPipelineOptions.class);
// default configuration from class path
Pipeline p = Pipeline.create();
@@ -58,7 +54,7 @@ public void testConfigProperties() throws Exception {
DAG dag = TestApexRunner.translate(p, options);
OperatorMeta t1Meta = dag.getOperatorMeta(operName);
Assert.assertNotNull(t1Meta);
- Assert.assertEquals(new Integer(32),
t1Meta.getValue(OperatorContext.MEMORY_MB));
+ Assert.assertEquals(Integer.valueOf(32),
t1Meta.getValue(OperatorContext.MEMORY_MB));
File tmp = File.createTempFile("beam-runners-apex-", ".properties");
tmp.deleteOnExit();
@@ -73,8 +69,7 @@ public void testConfigProperties() throws Exception {
t1Meta = dag.getOperatorMeta(operName);
Assert.assertNotNull(t1Meta);
- Assert.assertEquals(new Integer(64),
t1Meta.getValue(OperatorContext.MEMORY_MB));
-
+ Assert.assertEquals(Integer.valueOf(64),
t1Meta.getValue(OperatorContext.MEMORY_MB));
}
@Test
@@ -87,21 +82,20 @@ public void testParDoChaining() throws Exception {
ApexPipelineOptions options =
PipelineOptionsFactory.as(ApexPipelineOptions.class);
DAG dag = TestApexRunner.translate(p, options);
- String[] expectedThreadLocal = {
"/CreateActual/FilterActuals/Window.Assign" };
+ String[] expectedThreadLocal =
{"/CreateActual/FilterActuals/Window.Assign"};
Set<String> actualThreadLocal = Sets.newHashSet();
for (DAG.StreamMeta sm : dag.getAllStreamsMeta()) {
DAG.OutputPortMeta opm = sm.getSource();
if (sm.getLocality() == Locality.THREAD_LOCAL) {
- String name = opm.getOperatorMeta().getName();
- String prefix = "PAssert$";
- if (name.startsWith(prefix)) {
- // remove indeterministic prefix
- name = name.substring(prefix.length() + 1);
- }
- actualThreadLocal.add(name);
+ String name = opm.getOperatorMeta().getName();
+ String prefix = "PAssert$";
+ if (name.startsWith(prefix)) {
+ // remove indeterministic prefix
+ name = name.substring(prefix.length() + 1);
+ }
+ actualThreadLocal.add(name);
}
}
Assert.assertThat(actualThreadLocal,
Matchers.hasItems(expectedThreadLocal));
}
-
}
diff --git
a/runners/apex/src/test/java/org/apache/beam/runners/apex/ApexYarnLauncherTest.java
b/runners/apex/src/test/java/org/apache/beam/runners/apex/ApexYarnLauncherTest.java
index 5f91c19a219..c17b97ab3e2 100644
---
a/runners/apex/src/test/java/org/apache/beam/runners/apex/ApexYarnLauncherTest.java
+++
b/runners/apex/src/test/java/org/apache/beam/runners/apex/ApexYarnLauncherTest.java
@@ -128,7 +128,7 @@ public void testCreateJar() throws Exception {
Assert.assertTrue("exists: " + jarFile, jarFile.exists());
URI uri = URI.create("jar:" + jarFile.toURI());
final Map<String, ?> env = Collections.singletonMap("create", "true");
- try (final FileSystem zipfs = FileSystems.newFileSystem(uri, env);) {
+ try (final FileSystem zipfs = FileSystems.newFileSystem(uri, env)) {
Assert.assertTrue("manifest",
Files.isRegularFile(zipfs.getPath(JarFile.MANIFEST_NAME)));
Assert.assertTrue("file1", Files.isRegularFile(zipfs.getPath(file1)));
}
diff --git
a/runners/apex/src/test/java/org/apache/beam/runners/apex/examples/WordCountTest.java
b/runners/apex/src/test/java/org/apache/beam/runners/apex/examples/WordCountTest.java
index 65f6fbfd9b6..f07e07a5761 100644
---
a/runners/apex/src/test/java/org/apache/beam/runners/apex/examples/WordCountTest.java
+++
b/runners/apex/src/test/java/org/apache/beam/runners/apex/examples/WordCountTest.java
@@ -66,6 +66,7 @@ public void processElement(ProcessContext c) {
private static final long serialVersionUID = 1L;
private final Counter emptyLines = Metrics.counter("main", "emptyLines");
+ @SuppressWarnings("StringSplitter")
@ProcessElement
public void processElement(ProcessContext c) {
if (c.element().trim().isEmpty()) {
----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on GitHub and use the
URL above to go to the specific comment.
For queries about this service, please contact Infrastructure at:
[email protected]
Issue Time Tracking
-------------------
Worklog Id: (was: 106176)
Time Spent: 0.5h (was: 20m)
> Enforce ErrorProne analysis in apex project
> -------------------------------------------
>
> Key: BEAM-4306
> URL: https://issues.apache.org/jira/browse/BEAM-4306
> Project: Beam
> Issue Type: Improvement
> Components: runner-apex
> Reporter: Scott Wegner
> Assignee: Ismaël Mejía
> Priority: Minor
> Labels: errorprone, starter
> Time Spent: 0.5h
> Remaining Estimate: 0h
>
> Java ErrorProne static analysis was [recently
> enabled|https://github.com/apache/beam/pull/5161] in the Gradle build
> process, but only as warnings. ErrorProne errors are generally useful and
> easy to fix. Some work was done to [make sdks-java-core
> ErrorProne-clean|https://github.com/apache/beam/pull/5319] and add
> enforcement. This task is clean ErrorProne warnings and add enforcement in
> {{beam-runners-apex}}. Additional context discussed on the [dev
> list|https://lists.apache.org/thread.html/95aae2785c3cd728c2d3378cbdff2a7ba19caffcd4faa2049d2e2f46@%3Cdev.beam.apache.org%3E].
> Fixing this issue will involve:
> # Follow instructions in the [Contribution
> Guide|https://beam.apache.org/contribute/] to set up a {{beam}} development
> environment.
> # Run the following command to compile and run ErrorProne analysis on the
> project: {{./gradlew :beam-runners-apex:assemble}}
> # Fix each ErrorProne warning from the {{runners/apex}} project.
> # In {{runners/apex/build.gradle}}, add {{failOnWarning: true}} to the call
> the {{applyJavaNature()}}
> ([example|https://github.com/apache/beam/pull/5319/files#diff-9390c20635aed5f42f83b97506a87333R20]).
> This starter issue is sponsored by [~swegner]. Feel free to [reach
> out|https://beam.apache.org/community/contact-us/] with questions or code
> review:
> * JIRA: [~swegner]
> * GitHub: [@swegner|https://github.com/swegner]
> * Slack: [@Scott Wegner|https://s.apache.org/beam-slack-channel]
> * Email: swegner at google dot com
--
This message was sent by Atlassian JIRA
(v7.6.3#76005)