Skip to content

Commit 06a5e67

Browse files
authored
[BEAM-11936] Fix errorprone warnings (apache#15890)
* [BEAM-11936] Fix errorprone warning: FloatingPointAssertionWithinEpsilon * [BEAM-11936] Fix errorprone warning: LockNotBeforeTry * [BEAM-11936] Fix errorprone warning: PreferJavaTimeOverload * [BEAM-11936] Fix errorprone warning: ModifiedButNotUsed * [BEAM-11936] Fix errorprone warning: UnusedNestedClass * [BEAM-11936] Remove suppresswarnings * [BEAM-11936] Fix suppressed warnings * [BEAM-11936] Fix errorprone warnings after merge master * [BEAM-11936] Remove suppressWarnings errorprone: PreferJavaTimeOverload * [BEAM-11936] Remove suppressWarnings errorprone: PreferJavaTimeOverload * [BEAM-11936] Fix leftover warning errorprone: PreferJavaTimeOverload * [BEAM-11936] Fix leftover warning errorprone: PreferJavaTimeOverload * [BEAM-11936] Add suppresswarning * [BEAM-11936] Remove unused inner class
1 parent e52bdf2 commit 06a5e67

47 files changed

Lines changed: 66 additions & 596 deletions

File tree

Some content is hidden

Large Commits have some content hidden by default. Use the searchbox below for content that may be hidden.

buildSrc/src/main/groovy/org/apache/beam/gradle/BeamModulePlugin.groovy

Lines changed: 0 additions & 6 deletions
Original file line numberDiff line numberDiff line change
@@ -1140,18 +1140,12 @@ class BeamModulePlugin implements Plugin<Project> {
11401140
options.errorprone.errorproneArgs.add("-Xep:EqualsGetClass:OFF")
11411141
options.errorprone.errorproneArgs.add("-Xep:EqualsUnsafeCast:OFF")
11421142
options.errorprone.errorproneArgs.add("-Xep:ExtendsAutoValue:OFF")
1143-
options.errorprone.errorproneArgs.add("-Xep:FloatingPointAssertionWithinEpsilon:OFF")
11441143
options.errorprone.errorproneArgs.add("-Xep:JavaTimeDefaultTimeZone:OFF")
1145-
options.errorprone.errorproneArgs.add("-Xep:LockNotBeforeTry:OFF")
11461144
options.errorprone.errorproneArgs.add("-Xep:MixedMutabilityReturnType:OFF")
1147-
options.errorprone.errorproneArgs.add("-Xep:PreferJavaTimeOverload:OFF")
1148-
options.errorprone.errorproneArgs.add("-Xep:ModifiedButNotUsed:OFF")
11491145
options.errorprone.errorproneArgs.add("-Xep:ThreadPriorityCheck:OFF")
11501146
options.errorprone.errorproneArgs.add("-Xep:TimeUnitConversionChecker:OFF")
11511147
options.errorprone.errorproneArgs.add("-Xep:UndefinedEquals:OFF")
11521148
options.errorprone.errorproneArgs.add("-Xep:UnnecessaryLambda:OFF")
1153-
options.errorprone.errorproneArgs.add("-Xep:UnusedVariable:OFF")
1154-
options.errorprone.errorproneArgs.add("-Xep:UnusedNestedClass:OFF")
11551149
options.errorprone.errorproneArgs.add("-Xep:UnsafeReflectiveConstructionCast:OFF")
11561150
}
11571151

examples/java/src/main/java/org/apache/beam/examples/snippets/Snippets.java

Lines changed: 8 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -191,6 +191,7 @@ public static void modelBigQueryIO(
191191
}
192192

193193
{
194+
@SuppressWarnings("ModifiedButNotUsed")
194195
// [START BigQueryDataTypes]
195196
TableRow row = new TableRow();
196197
row.set("string", "abc");
@@ -1174,6 +1175,7 @@ public String toJson() {
11741175
}
11751176
}
11761177

1178+
@SuppressWarnings("unused")
11771179
private static class BundleFinalization {
11781180
private static class BundleFinalizationDoFn extends DoFn<String, Integer> {
11791181
// [START BundleFinalize]
@@ -1191,6 +1193,7 @@ public void processElement(ProcessContext c, BundleFinalizer bundleFinalizer) {
11911193
}
11921194
}
11931195

1196+
@SuppressWarnings("unused")
11941197
private static class SplittableDoFn {
11951198

11961199
private static void seekToNextRecordBoundaryInFile(
@@ -1230,6 +1233,7 @@ public Coder<OffsetRange> getRestrictionCoder() {
12301233
}
12311234
// [END SDF_BasicExample]
12321235

1236+
@SuppressWarnings("unused")
12331237
private static class BasicExampleWithInitialSplitting extends FileToWordsFn {
12341238
// [START SDF_BasicExampleWithSplitting]
12351239
void splitRestriction(
@@ -1248,6 +1252,7 @@ void splitRestriction(
12481252
// [END SDF_BasicExampleWithSplitting]
12491253
}
12501254

1255+
@SuppressWarnings("unused")
12511256
private static class BasicExampleWithBadTryClaimLoop extends DoFn<String, Integer> {
12521257
// [START SDF_BadTryClaimLoop]
12531258
@ProcessElement
@@ -1271,6 +1276,7 @@ public void badTryClaimLoop(
12711276
// [END SDF_BadTryClaimLoop]
12721277
}
12731278

1279+
@SuppressWarnings("unused")
12741280
private static class CustomWatermarkEstimatorExample extends DoFn<String, Integer> {
12751281
private static Instant currentWatermark = Instant.now();
12761282

@@ -1336,6 +1342,7 @@ public WatermarkEstimator<MyCustomWatermarkState> newWatermarkEstimator(
13361342
}
13371343
// [END SDF_CustomWatermarkEstimator]
13381344

1345+
@SuppressWarnings("unused")
13391346
private static class UserInitiatedCheckpointExample extends DoFn<String, Integer> {
13401347
public static class ThrottlingException extends Exception {}
13411348

@@ -1398,6 +1405,7 @@ TruncateResult<OffsetRange> truncateRestriction(
13981405
// [END SDF_Truncate]
13991406
}
14001407

1408+
@SuppressWarnings("unused")
14011409
private static class GetSizeExample extends DoFn<String, Integer> {
14021410
// [START SDF_GetSize]
14031411
@GetSize

runners/core-construction-java/src/test/java/org/apache/beam/runners/core/construction/PTransformMatchersTest.java

Lines changed: 0 additions & 21 deletions
Original file line numberDiff line numberDiff line change
@@ -57,9 +57,7 @@
5757
import org.apache.beam.sdk.transforms.ViewFn;
5858
import org.apache.beam.sdk.transforms.resourcehints.ResourceHints;
5959
import org.apache.beam.sdk.transforms.splittabledofn.RestrictionTracker;
60-
import org.apache.beam.sdk.transforms.windowing.BoundedWindow;
6160
import org.apache.beam.sdk.transforms.windowing.GlobalWindows;
62-
import org.apache.beam.sdk.transforms.windowing.PaneInfo;
6361
import org.apache.beam.sdk.transforms.windowing.Window;
6462
import org.apache.beam.sdk.values.KV;
6563
import org.apache.beam.sdk.values.PCollection;
@@ -74,7 +72,6 @@
7472
import org.apache.beam.sdk.values.WindowingStrategy;
7573
import org.apache.beam.vendor.guava.v26_0_jre.com.google.common.base.MoreObjects;
7674
import org.apache.beam.vendor.guava.v26_0_jre.com.google.common.collect.ImmutableMap;
77-
import org.checkerframework.checker.nullness.qual.Nullable;
7875
import org.hamcrest.Matchers;
7976
import org.joda.time.Duration;
8077
import org.junit.Rule;
@@ -598,22 +595,4 @@ public WriteOperation<Void, Integer> createWriteOperation() {
598595
ResourceHints.create(),
599596
p);
600597
}
601-
602-
private static class FakeFilenamePolicy extends FilenamePolicy {
603-
@Override
604-
public ResourceId windowedFilename(
605-
int shardNumber,
606-
int numShards,
607-
BoundedWindow window,
608-
PaneInfo paneInfo,
609-
FileBasedSink.OutputFileHints outputFileHints) {
610-
throw new UnsupportedOperationException("should not be called");
611-
}
612-
613-
@Override
614-
public @Nullable ResourceId unwindowedFilename(
615-
int shardNumber, int numShards, FileBasedSink.OutputFileHints outputFileHints) {
616-
throw new UnsupportedOperationException("should not be called");
617-
}
618-
}
619598
}

runners/direct-java/src/main/java/org/apache/beam/runners/direct/NanosOffsetClock.java

Lines changed: 2 additions & 4 deletions
Original file line numberDiff line numberDiff line change
@@ -17,7 +17,7 @@
1717
*/
1818
package org.apache.beam.runners.direct;
1919

20-
import java.util.concurrent.TimeUnit;
20+
import java.time.Duration;
2121
import org.joda.time.Instant;
2222

2323
/** A {@link Clock} that uses {@link System#nanoTime()} to track the progress of time. */
@@ -37,8 +37,6 @@ private NanosOffsetClock() {
3737
@Override
3838
public Instant now() {
3939
return new Instant(
40-
baseMillis
41-
+ TimeUnit.MILLISECONDS.convert(
42-
System.nanoTime() - nanosAtBaseMillis, TimeUnit.NANOSECONDS));
40+
baseMillis + Duration.ofNanos(System.nanoTime() - nanosAtBaseMillis).toMillis());
4341
}
4442
}

runners/direct-java/src/main/java/org/apache/beam/runners/direct/TransformEvaluatorRegistry.java

Lines changed: 0 additions & 16 deletions
Original file line numberDiff line numberDiff line change
@@ -39,7 +39,6 @@
3939
import java.util.Map;
4040
import java.util.concurrent.atomic.AtomicBoolean;
4141
import org.apache.beam.runners.core.SplittableParDoViaKeyedWorkItems;
42-
import org.apache.beam.runners.core.SplittableParDoViaKeyedWorkItems.ProcessElements;
4342
import org.apache.beam.runners.core.construction.PTransformTranslation;
4443
import org.apache.beam.runners.core.construction.PTransformTranslation.TransformPayloadTranslator;
4544
import org.apache.beam.runners.core.construction.TransformPayloadTranslatorRegistrar;
@@ -140,21 +139,6 @@ public static class DirectTransformsRegistrar implements TransformPayloadTransla
140139
}
141140
}
142141

143-
/**
144-
* A translator just to vend the URN. This will need to be moved to runners-core-construction-java
145-
* once SDF is reorganized appropriately.
146-
*/
147-
private static class SplittableParDoProcessElementsTranslator
148-
extends TransformPayloadTranslator.NotSerializable<ProcessElements<?, ?, ?, ?, ?>> {
149-
150-
private SplittableParDoProcessElementsTranslator() {}
151-
152-
@Override
153-
public String getUrn(ProcessElements<?, ?, ?, ?, ?> transform) {
154-
return SPLITTABLE_PROCESS_URN;
155-
}
156-
}
157-
158142
// the TransformEvaluatorFactories can construct instances of all generic types of transform,
159143
// so all instances of a primitive can be handled with the same evaluator factory.
160144
private final Map<String, TransformEvaluatorFactory> factories;

runners/direct-java/src/main/java/org/apache/beam/runners/direct/UnboundedReadDeduplicator.java

Lines changed: 1 addition & 2 deletions
Original file line numberDiff line numberDiff line change
@@ -17,7 +17,6 @@
1717
*/
1818
package org.apache.beam.runners.direct;
1919

20-
import java.util.concurrent.TimeUnit;
2120
import java.util.concurrent.atomic.AtomicBoolean;
2221
import org.apache.beam.runners.core.construction.SplittableParDo.PrimitiveUnboundedRead;
2322
import org.apache.beam.runners.local.StructuralKey;
@@ -74,7 +73,7 @@ public static UnboundedReadDeduplicator create() {
7473
private CachedIdDeduplicator() {
7574
ids =
7675
CacheBuilder.newBuilder()
77-
.expireAfterAccess(MAX_RETENTION_SINCE_ACCESS, TimeUnit.MILLISECONDS)
76+
.expireAfterAccess(java.time.Duration.ofMillis(MAX_RETENTION_SINCE_ACCESS))
7877
.maximumSize(100_000L)
7978
.build(new TrueBooleanLoader());
8079
}

runners/direct-java/src/test/java/org/apache/beam/runners/direct/CloningBundleFactoryTest.java

Lines changed: 0 additions & 67 deletions
Original file line numberDiff line numberDiff line change
@@ -36,9 +36,7 @@
3636
import org.apache.beam.sdk.coders.VarIntCoder;
3737
import org.apache.beam.sdk.testing.TestPipeline;
3838
import org.apache.beam.sdk.transforms.Create;
39-
import org.apache.beam.sdk.transforms.DoFn;
4039
import org.apache.beam.sdk.transforms.GroupByKey;
41-
import org.apache.beam.sdk.transforms.SimpleFunction;
4240
import org.apache.beam.sdk.transforms.WithKeys;
4341
import org.apache.beam.sdk.util.UserCodeException;
4442
import org.apache.beam.sdk.util.WindowedValue;
@@ -191,69 +189,4 @@ public Record decode(InputStream inStream) throws IOException {
191189
throw new CoderException("Decode not allowed");
192190
}
193191
}
194-
195-
private static class RecordStructuralValueCoder extends AtomicCoder<Record> {
196-
@Override
197-
public void encode(Record value, OutputStream outStream) throws CoderException, IOException {}
198-
199-
@Override
200-
public Record decode(InputStream inStream) throws CoderException, IOException {
201-
return new Record() {
202-
@Override
203-
public String toString() {
204-
return "DecodedRecord";
205-
}
206-
};
207-
}
208-
209-
@Override
210-
public boolean consistentWithEquals() {
211-
return true;
212-
}
213-
214-
@Override
215-
public Object structuralValue(Record value) {
216-
return value;
217-
}
218-
}
219-
220-
private static class RecordNotConsistentWithEqualsStructuralValueCoder
221-
extends AtomicCoder<Record> {
222-
@Override
223-
public void encode(Record value, OutputStream outStream) throws CoderException, IOException {}
224-
225-
@Override
226-
public Record decode(InputStream inStream) throws CoderException, IOException {
227-
return new Record() {
228-
@Override
229-
public String toString() {
230-
return "DecodedRecord";
231-
}
232-
};
233-
}
234-
235-
@Override
236-
public boolean consistentWithEquals() {
237-
return false;
238-
}
239-
240-
@Override
241-
public Object structuralValue(Record value) {
242-
return value;
243-
}
244-
}
245-
246-
private static class IdentityDoFn extends DoFn<Record, Record> {
247-
@ProcessElement
248-
public void proc(ProcessContext ctxt) {
249-
ctxt.output(ctxt.element());
250-
}
251-
}
252-
253-
private static class SimpleIdentity extends SimpleFunction<Record, Record> {
254-
@Override
255-
public Record apply(Record input) {
256-
return input;
257-
}
258-
}
259192
}

runners/flink/src/main/java/org/apache/beam/runners/flink/FlinkStreamingPipelineTranslator.java

Lines changed: 2 additions & 2 deletions
Original file line numberDiff line numberDiff line change
@@ -21,13 +21,13 @@
2121

2222
import java.io.IOException;
2323
import java.nio.ByteBuffer;
24+
import java.time.Duration;
2425
import java.util.Arrays;
2526
import java.util.HashMap;
2627
import java.util.List;
2728
import java.util.Map;
2829
import java.util.Objects;
2930
import java.util.concurrent.ThreadLocalRandom;
30-
import java.util.concurrent.TimeUnit;
3131
import org.apache.beam.runners.core.construction.PTransformReplacements;
3232
import org.apache.beam.runners.core.construction.PTransformTranslation;
3333
import org.apache.beam.runners.core.construction.ReplacementOutputs;
@@ -368,7 +368,7 @@ public ShardedKey<Integer> assignShardKey(
368368
cache =
369369
CacheBuilder.newBuilder()
370370
.maximumSize(CACHE_MAX_SIZE)
371-
.expireAfterAccess(CACHE_EXPIRE_SECONDS, TimeUnit.SECONDS)
371+
.expireAfterAccess(Duration.ofSeconds(CACHE_EXPIRE_SECONDS))
372372
.build();
373373
}
374374

runners/flink/src/main/java/org/apache/beam/runners/flink/FlinkStreamingTransformTranslators.java

Lines changed: 0 additions & 17 deletions
Original file line numberDiff line numberDiff line change
@@ -1417,23 +1417,6 @@ public void flatMap(
14171417
}
14181418
}
14191419

1420-
/**
1421-
* A translator just to vend the URN. This will need to be moved to runners-core-construction-java
1422-
* once SDF is reorganized appropriately.
1423-
*/
1424-
private static class SplittableParDoProcessElementsTranslator
1425-
extends PTransformTranslation.TransformPayloadTranslator.NotSerializable<
1426-
SplittableParDoViaKeyedWorkItems.ProcessElements<?, ?, ?, ?, ?>> {
1427-
1428-
private SplittableParDoProcessElementsTranslator() {}
1429-
1430-
@Override
1431-
public String getUrn(
1432-
SplittableParDoViaKeyedWorkItems.ProcessElements<?, ?, ?, ?, ?> transform) {
1433-
return SPLITTABLE_PROCESS_URN;
1434-
}
1435-
}
1436-
14371420
/** Registers classes specialized to the Flink runner. */
14381421
@AutoService(TransformPayloadTranslatorRegistrar.class)
14391422
public static class FlinkTransformsRegistrar implements TransformPayloadTranslatorRegistrar {

runners/flink/src/main/java/org/apache/beam/runners/flink/translation/wrappers/streaming/DoFnOperator.java

Lines changed: 1 addition & 1 deletion
Original file line numberDiff line numberDiff line change
@@ -1125,8 +1125,8 @@ public <T> void output(TupleTag<T> tag, WindowedValue<T> value) {
11251125
}
11261126

11271127
private void buffer(KV<Integer, WindowedValue<?>> taggedValue) {
1128+
bufferLock.lock();
11281129
try {
1129-
bufferLock.lock();
11301130
pushedBackElementsHandler.pushBack(taggedValue);
11311131
} catch (Exception e) {
11321132
throw new RuntimeException("Couldn't pushback element.", e);

0 commit comments

Comments
 (0)