Skip to content

Commit

Permalink
[CELEBORN-1190][FOLLOWUP] Fix WARNING of error prone
Browse files Browse the repository at this point in the history
### What changes were proposed in this pull request?

- Fix `WARNING` of error prone.
- Disable `EmptyCatch`, `JdkObsolete`, `MutableConstantField` and `UnnecessaryParentheses`.

### Why are the changes needed?

There are many `WARNING` generated by error prone. We should follow the suggestion of error prone to fix `WARNING`.

```
$ mvn clean install -DskipTests -Dcheckstyle.skip=true -Drat.skip=true -Dspotless.check.skip=true|grep WARNING|grep java
[WARNING] /Users/nicholas/Github/celeborn/common/src/main/java/org/apache/celeborn/common/network/sasl/SaslUtils.java:[44,25] [MutableConstantField] Constant field declarations should use the immutable type (such as ImmutableList) instead of the general collection interface type (such as List)
[WARNING] /Users/nicholas/Github/celeborn/common/src/main/java/org/apache/celeborn/common/network/sasl/SaslUtils.java:[47,18] [MutableConstantField] Constant field declarations should use the immutable type (such as ImmutableList) instead of the general collection interface type (such as List)
[WARNING] /Users/nicholas/Github/celeborn/common/src/main/java/org/apache/celeborn/common/network/client/TransportClientBootstrap.java:[34,5] [InvalidParam] Parameter name `channel` is unknown.
[WARNING] /Users/nicholas/Github/celeborn/common/src/main/java/org/apache/celeborn/common/network/client/TransportResponseHandler.java:[96,29] [StaticAssignmentInConstructor] This assignment is to a static field. Mutating static state from a constructor is highly error-prone.
[WARNING] /Users/nicholas/Github/celeborn/common/src/main/java/org/apache/celeborn/common/network/client/TransportResponseHandler.java:[104,30] [StaticAssignmentInConstructor] This assignment is to a static field. Mutating static state from a constructor is highly error-prone.
[WARNING] /Users/nicholas/Github/celeborn/common/src/main/java/org/apache/celeborn/common/network/sasl/anonymous/AnonymousSaslServerFactory.java:[67,2] [ClassCanBeStatic] Inner class is non-static but does not reference enclosing class
[WARNING] /Users/nicholas/Github/celeborn/common/src/main/java/org/apache/celeborn/common/meta/FileInfo.java:[60,17] [NonAtomicVolatileUpdate] This update of a volatile variable is non-atomic
[WARNING] /Users/nicholas/Github/celeborn/common/src/main/java/org/apache/celeborn/common/network/util/TransportFrameDecoder.java:[54,46] [JdkObsolete] It is very rare for LinkedList to out-perform ArrayList or ArrayDeque. Avoid it unless you're willing to invest a lot of time into benchmarking. Caveat: LinkedList supports null elements, but ArrayDeque does not.
[WARNING] /Users/nicholas/Github/celeborn/common/src/main/java/org/apache/celeborn/common/network/ssl/ReloadingX509TrustManager.java:[207,29] [NonAtomicVolatileUpdate] This update of a volatile variable is non-atomic
[WARNING] /Users/nicholas/Github/celeborn/common/src/main/java/org/apache/celeborn/common/network/ssl/ReloadingX509TrustManager.java:[216,28] [NonAtomicVolatileUpdate] This update of a volatile variable is non-atomic
[WARNING] /Users/nicholas/Github/celeborn/common/src/main/java/org/apache/celeborn/common/network/sasl/anonymous/AnonymousSaslClientFactory.java:[73,2] [ClassCanBeStatic] Inner class is non-static but does not reference enclosing class
[WARNING] /Users/nicholas/Github/celeborn/common/src/main/java/org/apache/celeborn/common/network/sasl/anonymous/AnonymousSaslClientFactory.java:[93,31] [DefaultCharset] Implicit use of the platform default charset, which can result in differing behaviour between JVM executions or incorrect behavior if the encoding of the data source doesn't match expectations.
[WARNING] /Users/nicholas/Github/celeborn/common/src/main/java/org/apache/celeborn/common/util/ExceptionUtils.java:[65,11] [UnnecessaryParentheses] These grouping parentheses are unnecessary; it is unlikely the code will be misinterpreted without them
[WARNING] /Users/nicholas/Github/celeborn/common/src/main/java/org/apache/celeborn/common/util/ExceptionUtils.java:[66,11] [UnnecessaryParentheses] These grouping parentheses are unnecessary; it is unlikely the code will be misinterpreted without them
[WARNING] /Users/nicholas/Github/celeborn/common/src/test/java/org/apache/celeborn/common/network/ssl/SslSampleConfigs.java:[164,16] [JavaUtilDate] Date has a bad API that leads to bugs; prefer java.time.Instant or LocalDate.
[WARNING] /Users/nicholas/Github/celeborn/common/src/test/java/org/apache/celeborn/common/network/ssl/SslSampleConfigs.java:[165,14] [JavaUtilDate] Date has a bad API that leads to bugs; prefer java.time.Instant or LocalDate.
[WARNING] /Users/nicholas/Github/celeborn/common/src/test/java/org/apache/celeborn/common/network/ssl/SslSampleConfigs.java:[165,35] [JavaUtilDate] Date has a bad API that leads to bugs; prefer java.time.Instant or LocalDate.
[WARNING] /Users/nicholas/Github/celeborn/common/src/test/java/org/apache/celeborn/common/network/SSLTransportClientFactorySuiteJ.java:[32,14] [MissingOverride] setUp overrides method in TransportClientFactorySuiteJ; expected Override
[WARNING] /Users/nicholas/Github/celeborn/common/src/test/java/org/apache/celeborn/common/network/SSLTransportClientFactorySuiteJ.java:[40,14] [MissingOverride] tearDown overrides method in TransportClientFactorySuiteJ; expected Override
[WARNING] /Users/nicholas/Github/celeborn/common/src/test/java/org/apache/celeborn/common/network/protocol/EncryptedMessageWithHeaderSuiteJ.java:[124,6] [UseCorrectAssertInTests] Java assert is used in test. For testing purposes Assert.* matchers should be used.
[WARNING] /Users/nicholas/Github/celeborn/common/src/test/java/org/apache/celeborn/common/network/RpcIntegrationSuiteJ.java:[255,15] [UnusedMethod] Private method 'assertErrorAndClosed' is never used.
[WARNING] /Users/nicholas/Github/celeborn/common/src/test/java/org/apache/celeborn/common/network/RpcIntegrationSuiteJ.java:[154,17] [UnusedNestedClass] This nested class is unused, and can be removed.
[WARNING] /Users/nicholas/Github/celeborn/common/src/test/java/org/apache/celeborn/common/network/RpcIntegrationSuiteJ.java:[57,15] [UnnecessaryParentheses] These grouping parentheses are unnecessary; it is unlikely the code will be misinterpreted without them
[WARNING] /Users/nicholas/Github/celeborn/common/src/test/java/org/apache/celeborn/common/network/ssl/ReloadingX509TrustManagerSuiteJ.java:[107,10] [AssertThrowsMultipleStatements] The lambda passed to assertThrows should contain exactly one statement
[WARNING] /Users/nicholas/Github/celeborn/common/src/test/java/org/apache/celeborn/common/network/ssl/ReloadingX509TrustManagerSuiteJ.java:[134,10] [AssertThrowsMultipleStatements] The lambda passed to assertThrows should contain exactly one statement
[WARNING] /Users/nicholas/Github/celeborn/client/src/main/java/org/apache/celeborn/client/read/LocalPartitionReader.java:[84,31] [StaticAssignmentInConstructor] This assignment is to a static field. Mutating static state from a constructor is highly error-prone.
[WARNING] /Users/nicholas/Github/celeborn/client/src/main/java/org/apache/celeborn/client/ShuffleClientImpl.java:[130,6] [ThreadLocalUsage] ThreadLocals should be stored in static fields
[WARNING] /Users/nicholas/Github/celeborn/client/src/main/java/org/apache/celeborn/client/ShuffleClientImpl.java:[714,6] [MissingCasesInEnumSwitch] Non-exhaustive switch; either add a default or handle the remaining cases: SUCCESS, PARTIAL_SUCCESS, REQUEST_FAILED, and 43 others
[WARNING] /Users/nicholas/Github/celeborn/client/src/main/java/org/apache/celeborn/client/ShuffleClientImpl.java:[1609,10] [MissingCasesInEnumSwitch] Non-exhaustive switch; either add a default or handle the remaining cases: PARTIAL_SUCCESS, REQUEST_FAILED, SHUFFLE_ALREADY_REGISTERED, and 45 others
[WARNING] /Users/nicholas/Github/celeborn/client/src/main/java/org/apache/celeborn/client/ShuffleClientImpl.java:[1648,26] [MissingOverride] updateFileGroup implements method in ShuffleClient; expected Override
[WARNING] /Users/nicholas/Github/celeborn/client/src/main/java/org/apache/celeborn/client/ShuffleClientImpl.java:[1654,57] [UnnecessaryParentheses] These grouping parentheses are unnecessary; it is unlikely the code will be misinterpreted without them
[WARNING] /Users/nicholas/Github/celeborn/client/src/main/java/org/apache/celeborn/client/ShuffleClientImpl.java:[1823,32] [MissingOverride] getDataClientFactory implements method in ShuffleClient; expected Override
[WARNING] /Users/nicholas/Github/celeborn/client/src/test/java/org/apache/celeborn/client/ShuffleClientSuiteJ.java:[185,6] [UseCorrectAssertInTests] Java assert is used in test. For testing purposes Assert.* matchers should be used.
[WARNING] /Users/nicholas/Github/celeborn/service/src/main/java/org/apache/celeborn/server/common/service/store/db/DbServiceManagerImpl.java:[70,33] [JavaUtilDate] Date has a bad API that leads to bugs; prefer java.time.Instant or LocalDate.
[WARNING] /Users/nicholas/Github/celeborn/service/src/main/java/org/apache/celeborn/server/common/service/store/db/DbServiceManagerImpl.java:[71,33] [JavaUtilDate] Date has a bad API that leads to bugs; prefer java.time.Instant or LocalDate.
[WARNING] /Users/nicholas/Github/celeborn/master/src/main/java/org/apache/celeborn/service/deploy/master/clustermeta/ha/HARaftServer.java:[424,11] [UnnecessaryParentheses] These grouping parentheses are unnecessary; it is unlikely the code will be misinterpreted without them
[WARNING] /Users/nicholas/Github/celeborn/master/src/main/java/org/apache/celeborn/service/deploy/master/clustermeta/ha/HARaftServer.java:[425,11] [UnnecessaryParentheses] These grouping parentheses are unnecessary; it is unlikely the code will be misinterpreted without them
[WARNING] /Users/nicholas/Github/celeborn/master/src/main/java/org/apache/celeborn/service/deploy/master/clustermeta/ha/HARaftServer.java:[496,55] [UnescapedEntity] This looks like a type with type parameters. The < and > characters here will be interpreted as HTML, which can be avoided by wrapping it in a {code } tag.
[WARNING] /Users/nicholas/Github/celeborn/master/src/main/java/org/apache/celeborn/service/deploy/master/clustermeta/SingleMasterMetaManager.java:[166,14] [MissingOverride] handleUpdatePartitionSize implements method in IMetadataHandler; expected Override
[WARNING] /Users/nicholas/Github/celeborn/master/src/main/java/org/apache/celeborn/service/deploy/master/SlotsAllocator.java:[298,61] [JdkObsolete] It is very rare for LinkedList to out-perform ArrayList or ArrayDeque. Avoid it unless you're willing to invest a lot of time into benchmarking. Caveat: LinkedList supports null elements, but ArrayDeque does not.
[WARNING] /Users/nicholas/Github/celeborn/worker/src/main/java/org/apache/celeborn/service/deploy/worker/storage/MapPartitionDataReader.java:[346,37] [NonAtomicVolatileUpdate] This update of a volatile variable is non-atomic
[WARNING] /Users/nicholas/Github/celeborn/worker/src/main/java/org/apache/celeborn/service/deploy/worker/memory/MemoryManager.java:[202,33] [SynchronizeOnNonFinalField] Synchronizing on non-final fields is not safe: if the field is ever updated, different threads may end up locking on different objects.
[WARNING] /Users/nicholas/Github/celeborn/worker/src/main/java/org/apache/celeborn/service/deploy/worker/memory/MemoryManager.java:[300,31] [UnnecessaryParentheses] These grouping parentheses are unnecessary; it is unlikely the code will be misinterpreted without them
[WARNING] /Users/nicholas/Github/celeborn/worker/src/main/java/org/apache/celeborn/service/deploy/worker/memory/MemoryManager.java:[497,17] [SynchronizeOnNonFinalField] Synchronizing on non-final fields is not safe: if the field is ever updated, different threads may end up locking on different objects.
[WARNING] /Users/nicholas/Github/celeborn/worker/src/main/java/org/apache/celeborn/service/deploy/worker/memory/MemoryManager.java:[503,17] [SynchronizeOnNonFinalField] Synchronizing on non-final fields is not safe: if the field is ever updated, different threads may end up locking on different objects.
[WARNING] /Users/nicholas/Github/celeborn/worker/src/main/java/org/apache/celeborn/service/deploy/worker/memory/MemoryManager.java:[513,39] [UnnecessaryParentheses] These grouping parentheses are unnecessary; it is unlikely the code will be misinterpreted without them
[WARNING] /Users/nicholas/Github/celeborn/worker/src/main/java/org/apache/celeborn/service/deploy/worker/storage/CreditStreamManager.java:[256,12] [ClassCanBeStatic] Inner class is non-static but does not reference enclosing class
[WARNING] /Users/nicholas/Github/celeborn/worker/src/main/java/org/apache/celeborn/service/deploy/worker/WorkerSecretRegistryImpl.java:[73,12] [CacheLoaderNull] The result of CacheLoader#load must be non-null.
[WARNING] /Users/nicholas/Github/celeborn/worker/src/main/java/org/apache/celeborn/service/deploy/worker/storage/ReducePartitionDataWriter.java:[69,13] [UnnecessaryParentheses] These grouping parentheses are unnecessary; it is unlikely the code will be misinterpreted without them
[WARNING] /Users/nicholas/Github/celeborn/worker/src/main/java/org/apache/celeborn/service/deploy/worker/storage/ReducePartitionDataWriter.java:[73,13] [UnnecessaryParentheses] These grouping parentheses are unnecessary; it is unlikely the code will be misinterpreted without them
[WARNING] /Users/nicholas/Github/celeborn/worker/src/main/java/org/apache/celeborn/service/deploy/worker/storage/ReducePartitionDataWriter.java:[103,24] [UnnecessaryParentheses] These grouping parentheses are unnecessary; it is unlikely the code will be misinterpreted without them
[WARNING] /Users/nicholas/Github/celeborn/worker/src/main/java/org/apache/celeborn/service/deploy/worker/storage/ReducePartitionDataWriter.java:[104,39] [UnnecessaryParentheses] These grouping parentheses are unnecessary; it is unlikely the code will be misinterpreted without them
[WARNING] /Users/nicholas/Github/celeborn/worker/src/main/java/org/apache/celeborn/service/deploy/worker/storage/MapPartitionDataWriter.java:[261,46] [ByteBufferBackingArray] ByteBuffer.array() shouldn't be called unless ByteBuffer.arrayOffset() is used or if the ByteBuffer was initialized using ByteBuffer.wrap() or ByteBuffer.allocate().
[WARNING] /Users/nicholas/Github/celeborn/worker/src/main/java/org/apache/celeborn/service/deploy/worker/storage/ChunkStreamManager.java:[102,40] [NonAtomicVolatileUpdate] This update of a volatile variable is non-atomic
[WARNING] /Users/nicholas/Github/celeborn/worker/src/main/java/org/apache/celeborn/service/deploy/worker/storage/ChunkStreamManager.java:[109,40] [NonAtomicVolatileUpdate] This update of a volatile variable is non-atomic
[WARNING] /Users/nicholas/Github/celeborn/worker/src/main/java/org/apache/celeborn/service/deploy/worker/storage/PartitionFilesSorter.java:[318,39] [IntLongMath] Expression of type int may overflow before being assigned to a long
[WARNING] /Users/nicholas/Github/celeborn/worker/src/test/java/org/apache/celeborn/service/deploy/worker/FetchHandlerSuiteJ.java:[133,6] [UnnecessaryParentheses] These grouping parentheses are unnecessary; it is unlikely the code will be misinterpreted without them
[WARNING] /Users/nicholas/Github/celeborn/worker/src/test/java/org/apache/celeborn/service/deploy/worker/network/SSLRequestTimeoutIntegrationSuiteJ.java:[32,14] [MissingOverride] setUp overrides method in RequestTimeoutIntegrationSuiteJ; expected Override
[WARNING] /Users/nicholas/Github/celeborn/worker/src/test/java/org/apache/celeborn/service/deploy/worker/network/SSLRequestTimeoutIntegrationSuiteJ.java:[40,14] [MissingOverride] tearDown overrides method in RequestTimeoutIntegrationSuiteJ; expected Override
[WARNING] /Users/nicholas/Github/celeborn/worker/src/test/java/org/apache/celeborn/service/deploy/worker/storage/ChunkFetchIntegrationSuiteJ.java:[74,15] [UnnecessaryParentheses] These grouping parentheses are unnecessary; it is unlikely the code will be misinterpreted without them
[WARNING] /Users/nicholas/Github/celeborn/worker/src/test/java/org/apache/celeborn/service/deploy/worker/storage/ChunkFetchIntegrationSuiteJ.java:[186,47] [JdkObsolete] It is very rare for LinkedList to out-perform ArrayList or ArrayDeque. Avoid it unless you're willing to invest a lot of time into benchmarking. Caveat: LinkedList supports null elements, but ArrayDeque does not.
[WARNING] /Users/nicholas/Github/celeborn/worker/src/test/java/org/apache/celeborn/service/deploy/worker/storage/SSLReducePartitionDataWriterSuiteJ.java:[30,26] [MissingOverride] createModuleTransportConf overrides method in DiskReducePartitionDataWriterSuiteJ; expected Override
[WARNING] /Users/nicholas/Github/celeborn/worker/src/test/java/org/apache/celeborn/service/deploy/worker/storage/local/DiskReducePartitionDataWriterSuiteJ.java:[234,47] [JdkObsolete] It is very rare for LinkedList to out-perform ArrayList or ArrayDeque. Avoid it unless you're willing to invest a lot of time into benchmarking. Caveat: LinkedList supports null elements, but ArrayDeque does not.
[WARNING] /Users/nicholas/Github/celeborn/worker/src/test/java/org/apache/celeborn/service/deploy/worker/storage/memory/MemoryReducePartitionDataWriterSuiteJ.java:[198,47] [JdkObsolete] It is very rare for LinkedList to out-perform ArrayList or ArrayDeque. Avoid it unless you're willing to invest a lot of time into benchmarking. Caveat: LinkedList supports null elements, but ArrayDeque does not.
```
```
$ mvn clean install -Pspark-2.4 -pl client-spark/common,client-spark/spark-2 -DskipTests -Dcheckstyle.skip=true -Drat.skip=true -Dspotless.check.skip=true|grep WARNING|grep java
[WARNING] /Users/nicholas/Github/celeborn/client-spark/common/src/main/java/org/apache/spark/shuffle/celeborn/SortBasedPusher.java:[109,57] [JdkObsolete] It is very rare for LinkedList to out-perform ArrayList or ArrayDeque. Avoid it unless you're willing to invest a lot of time into benchmarking. Caveat: LinkedList supports null elements, but ArrayDeque does not.
[WARNING] /Users/nicholas/Github/celeborn/client-spark/common/src/main/java/org/apache/spark/shuffle/celeborn/SendBufferPool.java:[56,14] [JdkObsolete] It is very rare for LinkedList to out-perform ArrayList or ArrayDeque. Avoid it unless you're willing to invest a lot of time into benchmarking. Caveat: LinkedList supports null elements, but ArrayDeque does not.
[WARNING] /Users/nicholas/Github/celeborn/client-spark/common/src/main/java/org/apache/spark/shuffle/celeborn/SendBufferPool.java:[57,21] [JdkObsolete] It is very rare for LinkedList to out-perform ArrayList or ArrayDeque. Avoid it unless you're willing to invest a lot of time into benchmarking. Caveat: LinkedList supports null elements, but ArrayDeque does not.
[WARNING] /Users/nicholas/Github/celeborn/client-spark/spark-2/src/main/java/org/apache/spark/shuffle/celeborn/SparkShuffleManager.java:[247,14] [UnusedMethod] Private method 'executorCores' is never used.
[WARNING] /Users/nicholas/Github/celeborn/client-spark/spark-2/src/main/java/org/apache/spark/shuffle/celeborn/SparkShuffleManager.java:[120,55] [ReferenceEquality] Comparison using reference equality instead of value equality
```
```
$ mvn clean install -Pspark-3.5 -pl client-spark/spark-3 -DskipTests -Dcheckstyle.skip=true -Drat.skip=true -Dspotless.check.skip=true|grep WARNING|grep java
[WARNING] /Users/nicholas/Github/celeborn/client-spark/spark-3/src/main/java/org/apache/spark/shuffle/celeborn/CelebornShuffleDataIO.java:[65,17] [MissingOverride] supportsReliableStorage implements method in ShuffleDriverComponents; expected Override
[WARNING] /Users/nicholas/Github/celeborn/client-spark/spark-3/src/main/java/org/apache/spark/shuffle/celeborn/SparkShuffleManager.java:[163,55] [ReferenceEquality] Comparison using reference equality instead of value equality
```
```
$ mvn clean install -Pflink-1.14 -pl client-flink/common,client-flink/flink-1.14 -DskipTests -Dcheckstyle.skip=true -Drat.skip=true -Dspotless.check.skip=true|grep WARNING|grep java
[WARNING] /Users/nicholas/Github/celeborn/client-flink/common/src/main/java/org/apache/celeborn/plugin/flink/readclient/CelebornBufferStream.java:[161,17] [SynchronizeOnNonFinalField] Synchronizing on non-final fields is not safe: if the field is ever updated, different threads may end up locking on different objects.
[WARNING] /Users/nicholas/Github/celeborn/client-flink/common/src/main/java/org/apache/celeborn/plugin/flink/readclient/CelebornBufferStream.java:[223,27] [SynchronizeOnNonFinalField] Synchronizing on non-final fields is not safe: if the field is ever updated, different threads may end up locking on different objects.
[WARNING] /Users/nicholas/Github/celeborn/client-flink/common/src/main/java/org/apache/celeborn/plugin/flink/ShuffleTaskInfo.java:[46,17] [SynchronizeOnNonFinalField] Synchronizing on non-final fields is not safe: if the field is ever updated, different threads may end up locking on different objects.
[WARNING] /Users/nicholas/Github/celeborn/client-flink/common/src/main/java/org/apache/celeborn/plugin/flink/RemoteShuffleInputGateDelegation.java:[99,66] [JdkObsolete] It is very rare for LinkedList to out-perform ArrayList or ArrayDeque. Avoid it unless you're willing to invest a lot of time into benchmarking. Caveat: LinkedList supports null elements, but ArrayDeque does not.
[WARNING] /Users/nicholas/Github/celeborn/client-flink/common/src/main/java/org/apache/celeborn/plugin/flink/RemoteShuffleInputGateDelegation.java:[236,21] [SynchronizeOnNonFinalField] Synchronizing on non-final fields is not safe: if the field is ever updated, different threads may end up locking on different objects.
[WARNING] /Users/nicholas/Github/celeborn/client-flink/common/src/main/java/org/apache/celeborn/plugin/flink/RemoteShuffleInputGateDelegation.java:[251,19] [SynchronizeOnNonFinalField] Synchronizing on non-final fields is not safe: if the field is ever updated, different threads may end up locking on different objects.
[WARNING] /Users/nicholas/Github/celeborn/client-flink/common/src/main/java/org/apache/celeborn/plugin/flink/RemoteShuffleInputGateDelegation.java:[267,17] [SynchronizeOnNonFinalField] Synchronizing on non-final fields is not safe: if the field is ever updated, different threads may end up locking on different objects.
[WARNING] /Users/nicholas/Github/celeborn/client-flink/common/src/main/java/org/apache/celeborn/plugin/flink/RemoteShuffleInputGateDelegation.java:[354,17] [SynchronizeOnNonFinalField] Synchronizing on non-final fields is not safe: if the field is ever updated, different threads may end up locking on different objects.
[WARNING] /Users/nicholas/Github/celeborn/client-flink/common/src/main/java/org/apache/celeborn/plugin/flink/RemoteShuffleInputGateDelegation.java:[392,17] [SynchronizeOnNonFinalField] Synchronizing on non-final fields is not safe: if the field is ever updated, different threads may end up locking on different objects.
[WARNING] /Users/nicholas/Github/celeborn/client-flink/common/src/main/java/org/apache/celeborn/plugin/flink/RemoteShuffleInputGateDelegation.java:[473,17] [SynchronizeOnNonFinalField] Synchronizing on non-final fields is not safe: if the field is ever updated, different threads may end up locking on different objects.
[WARNING] /Users/nicholas/Github/celeborn/client-flink/common/src/main/java/org/apache/celeborn/plugin/flink/RemoteShuffleInputGateDelegation.java:[533,17] [SynchronizeOnNonFinalField] Synchronizing on non-final fields is not safe: if the field is ever updated, different threads may end up locking on different objects.
[WARNING] /Users/nicholas/Github/celeborn/client-flink/common/src/main/java/org/apache/celeborn/plugin/flink/buffer/TransferBufferPool.java:[182,33] [MixedMutabilityReturnType] This method returns both mutable and immutable collections or maps from different paths. This may be confusing for users of the method.
[WARNING] /Users/nicholas/Github/celeborn/client-flink/common/src/main/java/org/apache/celeborn/plugin/flink/utils/FlinkUtils.java:[34,6] [DoubleBraceInitialization] Prefer collection factory methods or builders to the double-brace initialization pattern.
[WARNING] /Users/nicholas/Github/celeborn/client-flink/common/src/test/java/org/apache/celeborn/plugin/flink/BufferPackSuiteJ.java:[207,6] [CatchAndPrintStackTrace] Logging or rethrowing exceptions should usually be preferred to catching and calling printStackTrace
[WARNING] /Users/nicholas/Github/celeborn/client-flink/flink-1.14/src/test/java/org/apache/celeborn/plugin/flink/RemoteShuffleResultPartitionSuiteJ.java:[140,67] [CanonicalDuration] Duration can be expressed more clearly with different units
```
```
$ mvn clean install -Pflink-1.15 -pl client-flink/flink-1.15 -DskipTests -Dcheckstyle.skip=true -Drat.skip=true -Dspotless.check.skip=true|grep WARNING|grep java
[WARNING] /Users/nicholas/Github/celeborn/client-flink/flink-1.15/src/test/java/org/apache/celeborn/plugin/flink/RemoteShuffleResultPartitionSuiteJ.java:[140,67] [CanonicalDuration] Duration can be expressed more clearly with different units
```
```
$ mvn clean install -Pflink-1.17 -pl client-flink/flink-1.16 -DskipTests -Dcheckstyle.skip=true -Drat.skip=true -Dspotless.check.skip=true|grep WARNING|grep java
```
```
$ mvn clean install -Pflink-1.17 -pl client-flink/flink-1.17 -DskipTests -Dcheckstyle.skip=true -Drat.skip=true -Dspotless.check.skip=true|grep WARNING|grep java
[WARNING] /Users/nicholas/Github/celeborn/client-flink/flink-1.17/src/test/java/org/apache/celeborn/plugin/flink/RemoteShuffleResultPartitionSuiteJ.java:[140,67] [CanonicalDuration] Duration can be expressed more clearly with different units
```
```
$ mvn clean install -Pflink-1.18 -pl client-flink/flink-1.18 -DskipTests -Dcheckstyle.skip=true -Drat.skip=true -Dspotless.check.skip=true|grep WARNING|grep java
[WARNING] /Users/nicholas/Github/celeborn/client-flink/flink-1.18/src/test/java/org/apache/celeborn/plugin/flink/RemoteShuffleResultPartitionSuiteJ.java:[140,67] [CanonicalDuration] Duration can be expressed more clearly with different units
```
```
$ mvn clean install -Pflink-1.19 -pl client-flink/flink-1.19 -DskipTests -Dcheckstyle.skip=true -Drat.skip=true -Dspotless.check.skip=true|grep WARNING|grep java
[WARNING] /Users/nicholas/Github/celeborn/client-flink/flink-1.19/src/test/java/org/apache/celeborn/plugin/flink/RemoteShuffleResultPartitionSuiteJ.java:[140,67] [CanonicalDuration] Duration can be expressed more clearly with different units
```
```
$ mvn clean install -Pmr -pl client-mr/mr -DskipTests -Dcheckstyle.skip=true -Drat.skip=true -Dspotless.check.skip=true|grep WARNING|grep java
```

### Does this PR introduce _any_ user-facing change?

No.

### How was this patch tested?

Manual test.

```
$ mvn clean install -DskipTests -Dcheckstyle.skip=true -Drat.skip=true -Dspotless.check.skip=true|grep WARNING|grep java
$ mvn clean install -Pspark-2.4 -pl client-spark/common,client-spark/spark-2 -DskipTests -Dcheckstyle.skip=true -Drat.skip=true -Dspotless.check.skip=true|grep WARNING|grep java
$ mvn clean install -Pspark-3.5 -pl client-spark/spark-3 -DskipTests -Dcheckstyle.skip=true -Drat.skip=true -Dspotless.check.skip=true|grep WARNING|grep java
$ mvn clean install -Pflink-1.14 -pl client-flink/common,client-flink/flink-1.14 -DskipTests -Dcheckstyle.skip=true -Drat.skip=true -Dspotless.check.skip=true|grep WARNING|grep java
$ mvn clean install -Pflink-1.15 -pl client-flink/flink-1.15 -DskipTests -Dcheckstyle.skip=true -Drat.skip=true -Dspotless.check.skip=true|grep WARNING|grep java
$ mvn clean install -Pflink-1.16 -pl client-flink/flink-1.15 -DskipTests -Dcheckstyle.skip=true -Drat.skip=true -Dspotless.check.skip=true|grep WARNING|grep java
$ mvn clean install -Pflink-1.17 -pl client-flink/flink-1.17 -DskipTests -Dcheckstyle.skip=true -Drat.skip=true -Dspotless.check.skip=true|grep WARNING|grep java
$ mvn clean install -Pflink-1.18 -pl client-flink/flink-1.18 -DskipTests -Dcheckstyle.skip=true -Drat.skip=true -Dspotless.check.skip=true|grep WARNING|grep java
$ mvn clean install -Pflink-1.19 -pl client-flink/flink-1.19 -DskipTests -Dcheckstyle.skip=true -Drat.skip=true -Dspotless.check.skip=true|grep WARNING|grep java
$ mvn clean install -Pmr -pl client-mr/mr -DskipTests -Dcheckstyle.skip=true -Drat.skip=true -Dspotless.check.skip=true|grep WARNING|grep java
```

Closes apache#2555 from SteNicholas/CELEBORN-1190.

Authored-by: SteNicholas <programgeek@163.com>
Signed-off-by: Mridul Muralidharan <mridul<at>gmail.com>
  • Loading branch information
SteNicholas authored and Mridul Muralidharan committed Jul 25, 2024
1 parent 0a68ae0 commit c0ca952
Show file tree
Hide file tree
Showing 47 changed files with 175 additions and 225 deletions.
Original file line number Diff line number Diff line change
Expand Up @@ -60,7 +60,7 @@
public class RemoteShuffleInputGateDelegation {
private static final Logger LOG = LoggerFactory.getLogger(RemoteShuffleInputGateDelegation.class);
/** Lock to protect {@link #receivedBuffers} and {@link #cause} and {@link #closed}. */
private Object lock = new Object();
private final Object lock = new Object();

/** Name of the corresponding computing task. */
private String taskName;
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -33,7 +33,7 @@ public class ShuffleTaskInfo {
private ConcurrentHashMap<Integer, ConcurrentHashMap<Integer, AtomicInteger>>
shuffleIdMapAttemptIdIndex = JavaUtils.newConcurrentHashMap();
// task shuffle id -> celeborn shuffle id
private ConcurrentHashMap<String, Integer> taskShuffleIdToShuffleId =
private final ConcurrentHashMap<String, Integer> taskShuffleIdToShuffleId =
JavaUtils.newConcurrentHashMap();
// celeborn shuffle id -> task shuffle id
private ConcurrentHashMap<Integer, String> shuffleIdToTaskShuffleId =
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -182,16 +182,14 @@ private int assignCredits(CreditListener creditListener) {
private List<CreditAssignment> dispatchReservedCredits() {
assert Thread.holdsLock(lock);

if (numAvailableBuffers < MIN_CREDITS_TO_NOTIFY || listeners.size() <= 0) {
return Collections.emptyList();
}

List<CreditAssignment> creditAssignments = new ArrayList<>();
while (numAvailableBuffers > 0 && listeners.size() > 0) {
CreditListener creditListener = listeners.poll();
int numCredits = assignCredits(creditListener);
if (numCredits > 0) {
creditAssignments.add(new CreditAssignment(numCredits, creditListener));
if (numAvailableBuffers >= MIN_CREDITS_TO_NOTIFY && !listeners.isEmpty()) {
while (numAvailableBuffers > 0 && !listeners.isEmpty()) {
CreditListener creditListener = listeners.poll();
int numCredits = assignCredits(creditListener);
if (numCredits > 0) {
creditAssignments.add(new CreditAssignment(numCredits, creditListener));
}
}
}
return creditAssignments;
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -55,7 +55,7 @@ public class CelebornBufferStream {
private FlinkShuffleClientImpl mapShuffleClient;
private boolean isClosed;
private boolean isOpenSuccess;
private Object lock = new Object();
private final Object lock = new Object();
private Supplier<ByteBuf> bufferSupplier;
private int initialCredit;
private Consumer<RequestMessage> messageConsumer;
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -17,10 +17,10 @@

package org.apache.celeborn.plugin.flink.utils;

import java.util.HashSet;
import java.util.Map;
import java.util.Set;

import com.google.common.collect.ImmutableSet;
import org.apache.flink.api.common.JobID;
import org.apache.flink.configuration.Configuration;
import org.apache.flink.runtime.executiongraph.ExecutionAttemptID;
Expand All @@ -31,19 +31,16 @@
public class FlinkUtils {
private static final JobID ZERO_JOB_ID = new JobID(0, 0);
public static final Set<String> pluginConfNames =
new HashSet<String>() {
{
add("remote-shuffle.job.min.memory-per-partition");
add("remote-shuffle.job.min.memory-per-gate");
add("remote-shuffle.job.concurrent-readings-per-gate");
add("remote-shuffle.job.memory-per-partition");
add("remote-shuffle.job.memory-per-gate");
add("remote-shuffle.job.support-floating-buffer-per-input-gate");
add("remote-shuffle.job.enable-data-compression");
add("remote-shuffle.job.support-floating-buffer-per-output-gate");
add("remote-shuffle.job.compression.codec");
}
};
ImmutableSet.of(
"remote-shuffle.job.min.memory-per-partition",
"remote-shuffle.job.min.memory-per-gate",
"remote-shuffle.job.concurrent-readings-per-gate",
"remote-shuffle.job.memory-per-partition",
"remote-shuffle.job.memory-per-gate",
"remote-shuffle.job.support-floating-buffer-per-input-gate",
"remote-shuffle.job.enable-data-compression",
"remote-shuffle.job.support-floating-buffer-per-output-gate",
"remote-shuffle.job.compression.codec");

public static CelebornConf toCelebornConf(Configuration configuration) {
CelebornConf tmpCelebornConf = new CelebornConf();
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -203,10 +203,7 @@ public void testFailedToHandleRipeBufferAndClose() throws Exception {
packer.process(buffers.get(0), 0);
try {
packer.drain();
} catch (RuntimeException e) {
e.printStackTrace();
} catch (Exception e) {
throw e;
} catch (RuntimeException ignored) {
}

// this should never throw any exception
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -137,7 +137,7 @@ public void tesSimpleFlush() throws IOException, InterruptedException {

private List<SupplierWithException<BufferPool, IOException>> createBufferPoolFactory() {
NetworkBufferPool networkBufferPool =
new NetworkBufferPool(256 * 8, 32 * 1024, Duration.ofMillis(1000));
new NetworkBufferPool(256 * 8, 32 * 1024, Duration.ofSeconds(1));

int numBuffersPerPartition = 64 * 1024 / 32;
int numForResultPartition = numBuffersPerPartition * 7 / 8;
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -137,7 +137,7 @@ public void tesSimpleFlush() throws IOException, InterruptedException {

private List<SupplierWithException<BufferPool, IOException>> createBufferPoolFactory() {
NetworkBufferPool networkBufferPool =
new NetworkBufferPool(256 * 8, 32 * 1024, Duration.ofMillis(1000));
new NetworkBufferPool(256 * 8, 32 * 1024, Duration.ofSeconds(1));

int numBuffersPerPartition = 64 * 1024 / 32;
int numForResultPartition = numBuffersPerPartition * 7 / 8;
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -137,7 +137,7 @@ public void tesSimpleFlush() throws IOException, InterruptedException {

private List<SupplierWithException<BufferPool, IOException>> createBufferPoolFactory() {
NetworkBufferPool networkBufferPool =
new NetworkBufferPool(256 * 8, 32 * 1024, Duration.ofMillis(1000));
new NetworkBufferPool(256 * 8, 32 * 1024, Duration.ofSeconds(1));

int numBuffersPerPartition = 64 * 1024 / 32;
int numForResultPartition = numBuffersPerPartition * 7 / 8;
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -137,7 +137,7 @@ public void tesSimpleFlush() throws IOException, InterruptedException {

private List<SupplierWithException<BufferPool, IOException>> createBufferPoolFactory() {
NetworkBufferPool networkBufferPool =
new NetworkBufferPool(256 * 8, 32 * 1024, Duration.ofMillis(1000));
new NetworkBufferPool(256 * 8, 32 * 1024, Duration.ofSeconds(1));

int numBuffersPerPartition = 64 * 1024 / 32;
int numForResultPartition = numBuffersPerPartition * 7 / 8;
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -137,7 +137,7 @@ public void tesSimpleFlush() throws IOException, InterruptedException {

private List<SupplierWithException<BufferPool, IOException>> createBufferPoolFactory() {
NetworkBufferPool networkBufferPool =
new NetworkBufferPool(256 * 8, 32 * 1024, Duration.ofMillis(1000));
new NetworkBufferPool(256 * 8, 32 * 1024, Duration.ofSeconds(1));

int numBuffersPerPartition = 64 * 1024 / 32;
int numForResultPartition = numBuffersPerPartition * 7 / 8;
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -117,7 +117,7 @@ public <K, V, C> ShuffleHandle registerShuffle(

lifecycleManager.registerAppShuffleDeterminate(
shuffleId,
dependency.rdd().getOutputDeterministicLevel() != DeterministicLevel.INDETERMINATE());
!DeterministicLevel.INDETERMINATE().equals(dependency.rdd().getOutputDeterministicLevel()));

if (fallbackPolicyRunner.applyAllFallbackPolicy(
lifecycleManager, dependency.partitioner().numPartitions())) {
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -62,6 +62,7 @@ public CelebornShuffleDriverComponents(CelebornConf celebornConf) {
}

// Omitting @Override annotation to avoid compile error before Spark 3.5.0
@SuppressWarnings("MissingOverride")
public boolean supportsReliableStorage() {
return supportsReliableStorage;
}
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -160,7 +160,7 @@ public <K, V, C> ShuffleHandle registerShuffle(

lifecycleManager.registerAppShuffleDeterminate(
shuffleId,
dependency.rdd().getOutputDeterministicLevel() != DeterministicLevel.INDETERMINATE());
!DeterministicLevel.INDETERMINATE().equals(dependency.rdd().getOutputDeterministicLevel()));

if (fallbackPolicyRunner.applyAllFallbackPolicy(
lifecycleManager, dependency.partitioner().numPartitions())) {
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -126,6 +126,7 @@ public class ShuffleClientImpl extends ShuffleClient {
private final boolean authEnabled;
private final TransportConf dataTransportConf;

@SuppressWarnings("ThreadLocalUsage")
private final ThreadLocal<Compressor> compressorThreadLocal =
new ThreadLocal<Compressor>() {
@Override
Expand Down Expand Up @@ -747,6 +748,7 @@ void excludeWorkerByCause(StatusCode cause, PartitionLocation oldLocation) {
case PUSH_DATA_TIMEOUT_REPLICA:
pushExcludedWorkers.add(oldLocation.getPeer().hostAndPushPort());
break;
default: // fall out
}
}
}
Expand Down Expand Up @@ -1656,6 +1658,8 @@ protected Tuple2<ReduceFileGroups, String> loadFileGroupInternal(int shuffleId)
"Request %s return %s for %s.",
getReducerFileGroup, response.status(), shuffleId);
logger.warn(exceptionMsg);
break;
default: // fall out
}
}
} catch (Exception e) {
Expand All @@ -1666,6 +1670,7 @@ protected Tuple2<ReduceFileGroups, String> loadFileGroupInternal(int shuffleId)
}
}

@Override
public ReduceFileGroups updateFileGroup(int shuffleId, int partitionId)
throws CelebornIOException {
Tuple2<ReduceFileGroups, String> fileGroupTuple =
Expand Down Expand Up @@ -1850,6 +1855,7 @@ private StatusCode getPushDataFailCause(String message) {
}

@VisibleForTesting
@Override
public TransportClientFactory getDataClientFactory() {
return dataClientFactory;
}
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -69,6 +69,7 @@ public class LocalPartitionReader implements PartitionReader {
private TransportClient client;
private MetricsCallback metricsCallback;

@SuppressWarnings("StaticAssignmentInConstructor")
public LocalPartitionReader(
CelebornConf conf,
String shuffleKey,
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -18,6 +18,8 @@
package org.apache.celeborn.client;

import static org.junit.Assert.assertEquals;
import static org.junit.Assert.assertThrows;
import static org.junit.Assert.assertTrue;
import static org.mockito.ArgumentMatchers.any;
import static org.mockito.ArgumentMatchers.anyLong;
import static org.mockito.Mockito.mock;
Expand Down Expand Up @@ -170,59 +172,31 @@ public void testMergeData() throws IOException, InterruptedException {

@Test
public void testRegisterShuffleFailed() throws IOException, InterruptedException {
setupEnv(CompressionCodec.NONE, StatusCode.SLOT_NOT_AVAILABLE);
try {
shuffleClient.pushData(
TEST_SHUFFLE_ID,
TEST_ATTEMPT_ID,
TEST_ATTEMPT_ID,
TEST_REDUCRE_ID,
TEST_BUF1,
0,
TEST_BUF1.length,
1,
1);
assert false;
} catch (CelebornIOException e) {
assert e.getMessage()
.contains("Register shuffle failed for shuffle 1, reason: SLOT_NOT_AVAILABLE");
}

setupEnv(CompressionCodec.NONE, StatusCode.RESERVE_SLOTS_FAILED);
try {
shuffleClient.pushData(
TEST_SHUFFLE_ID,
TEST_ATTEMPT_ID,
TEST_ATTEMPT_ID,
TEST_REDUCRE_ID,
TEST_BUF1,
0,
TEST_BUF1.length,
1,
1);
assert false;
} catch (CelebornIOException e) {
assert e.getMessage()
.contains("Register shuffle failed for shuffle 1, reason: RESERVE_SLOTS_FAILED");
}
verifyRegisterShuffleFailure(StatusCode.SLOT_NOT_AVAILABLE);
verifyRegisterShuffleFailure(StatusCode.RESERVE_SLOTS_FAILED);
verifyRegisterShuffleFailure(StatusCode.REQUEST_FAILED);
}

setupEnv(CompressionCodec.NONE, StatusCode.REQUEST_FAILED);
try {
shuffleClient.pushData(
TEST_SHUFFLE_ID,
TEST_ATTEMPT_ID,
TEST_ATTEMPT_ID,
TEST_REDUCRE_ID,
TEST_BUF1,
0,
TEST_BUF1.length,
1,
1);
assert false;
} catch (CelebornIOException e) {
assert e.getMessage()
.contains("Register shuffle failed for shuffle 1, reason: REQUEST_FAILED");
}
private void verifyRegisterShuffleFailure(StatusCode statusCode)
throws IOException, InterruptedException {
setupEnv(CompressionCodec.NONE, statusCode);
CelebornIOException e =
assertThrows(
CelebornIOException.class,
() ->
shuffleClient.pushData(
TEST_SHUFFLE_ID,
TEST_ATTEMPT_ID,
TEST_ATTEMPT_ID,
TEST_REDUCRE_ID,
TEST_BUF1,
0,
TEST_BUF1.length,
1,
1));
assertTrue(
e.getMessage()
.contains("Register shuffle failed for shuffle 1, reason: " + statusCode.name()));
}

private CelebornConf setupEnv(CompressionCodec codec) throws IOException, InterruptedException {
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -56,7 +56,7 @@ public long getFileLength() {
return bytesFlushed;
}

public void updateBytesFlushed(long bytes) {
public synchronized void updateBytesFlushed(long bytes) {
bytesFlushed += bytes;
if (isReduceFileMeta) {
getReduceFileMeta().updateChunkOffset(bytesFlushed, false);
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -31,7 +31,6 @@ public interface TransportClientBootstrap {
* Performs the bootstrapping operation, throwing an exception on failure.
*
* @param client the transport client to bootstrap
* @param channel the associated channel with the transport client
* @throws RuntimeException
*/
void doBootstrap(TransportClient client) throws RuntimeException;
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -70,6 +70,7 @@ public class TransportResponseHandler extends MessageHandler<ResponseMessage> {
private static ScheduledExecutorService fetchTimeoutChecker = null;
private ScheduledFuture fetchCheckerScheduleFuture;

@SuppressWarnings("StaticAssignmentInConstructor")
public TransportResponseHandler(TransportConf conf, Channel channel) {
this.conf = conf;
this.channel = channel;
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -55,6 +55,7 @@ public boolean needCopyOut() {
return false;
}

@SuppressWarnings("NonOverridingEquals")
protected boolean equals(Message other) {
return Objects.equals(body, other.body);
}
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -19,6 +19,7 @@

import static org.apache.celeborn.common.network.sasl.SaslUtils.*;

import java.nio.charset.StandardCharsets;
import java.util.Map;

import javax.security.auth.callback.CallbackHandler;
Expand Down Expand Up @@ -70,7 +71,7 @@ public String[] getMechanismNames(Map<String, ?> props) {
return new String[] {ANONYMOUS};
}

class CelebornAnonymousSaslClient implements SaslClient {
static class CelebornAnonymousSaslClient implements SaslClient {

private boolean isCompleted = false;

Expand All @@ -90,7 +91,7 @@ public byte[] evaluateChallenge(byte[] challenge) throws SaslException {
throw new IllegalStateException("Authentication has already completed.");
}
isCompleted = true;
return ANONYMOUS.getBytes();
return ANONYMOUS.getBytes(StandardCharsets.UTF_8);
}

@Override
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -64,7 +64,7 @@ public String[] getMechanismNames(Map<String, ?> props) {
return new String[] {ANONYMOUS};
}

class CelebornAnonymousSaslServer implements SaslServer {
static class CelebornAnonymousSaslServer implements SaslServer {
private boolean isCompleted = false;

@Override
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -191,6 +191,7 @@ X509TrustManager loadTrustManager() throws IOException, GeneralSecurityException
return trustManager;
}

@SuppressWarnings("NonAtomicVolatileUpdate")
@Override
public void run() {
boolean running = true;
Expand Down
Loading

0 comments on commit c0ca952

Please sign in to comment.