This is an automated email from the ASF dual-hosted git repository.
chesnay pushed a commit to branch master
in repository https://gitbox.apache.org/repos/asf/flink.git
The following commit(s) were added to refs/heads/master by this push:
new 167d5640b24 [FLINK-32467][rpc] Move common RPC utils to rpc-core
167d5640b24 is described below
commit 167d5640b24c7ad08b864df1d271ebae8ba6d9dd
Author: Chesnay Schepler <[email protected]>
AuthorDate: Fri Jun 30 10:43:18 2023 +0200
[FLINK-32467][rpc] Move common RPC utils to rpc-core
---
.../java/org/apache/flink/runtime/rpc/akka/AkkaRpcSystemLoader.java | 1 +
.../apache/flink/runtime/rpc/akka/FallbackAkkaRpcSystemLoader.java | 1 +
.../concurrent/akka/ActorSystemScheduledExecutorAdapter.java | 1 +
.../org/apache/flink/runtime/rpc/akka/AkkaInvocationHandler.java | 2 +-
.../main/java/org/apache/flink/runtime/rpc/akka/AkkaRpcActor.java | 2 +-
.../main/java/org/apache/flink/runtime/rpc/akka/AkkaRpcService.java | 6 +++---
.../flink/runtime/rpc/akka/ContextClassLoadingSettingTest.java | 2 +-
.../org/apache/flink/runtime/concurrent}/ClassLoadingUtils.java | 2 +-
.../java/org/apache/flink/runtime/rpc}/CleanupOnCloseRpcSystem.java | 6 ++----
.../org/apache/flink/runtime/concurrent}/ClassLoadingUtilsTest.java | 2 +-
10 files changed, 13 insertions(+), 12 deletions(-)
diff --git
a/flink-rpc/flink-rpc-akka-loader/src/main/java/org/apache/flink/runtime/rpc/akka/AkkaRpcSystemLoader.java
b/flink-rpc/flink-rpc-akka-loader/src/main/java/org/apache/flink/runtime/rpc/akka/AkkaRpcSystemLoader.java
index a83defe0a4a..8796beed58d 100644
---
a/flink-rpc/flink-rpc-akka-loader/src/main/java/org/apache/flink/runtime/rpc/akka/AkkaRpcSystemLoader.java
+++
b/flink-rpc/flink-rpc-akka-loader/src/main/java/org/apache/flink/runtime/rpc/akka/AkkaRpcSystemLoader.java
@@ -20,6 +20,7 @@ package org.apache.flink.runtime.rpc.akka;
import org.apache.flink.configuration.Configuration;
import org.apache.flink.configuration.ConfigurationUtils;
import org.apache.flink.core.classloading.SubmoduleClassLoader;
+import org.apache.flink.runtime.rpc.CleanupOnCloseRpcSystem;
import org.apache.flink.runtime.rpc.RpcSystem;
import org.apache.flink.runtime.rpc.RpcSystemLoader;
import org.apache.flink.runtime.rpc.exceptions.RpcLoaderException;
diff --git
a/flink-rpc/flink-rpc-akka-loader/src/test/java/org/apache/flink/runtime/rpc/akka/FallbackAkkaRpcSystemLoader.java
b/flink-rpc/flink-rpc-akka-loader/src/test/java/org/apache/flink/runtime/rpc/akka/FallbackAkkaRpcSystemLoader.java
index 9c301ee294d..cdafd1ccd74 100644
---
a/flink-rpc/flink-rpc-akka-loader/src/test/java/org/apache/flink/runtime/rpc/akka/FallbackAkkaRpcSystemLoader.java
+++
b/flink-rpc/flink-rpc-akka-loader/src/test/java/org/apache/flink/runtime/rpc/akka/FallbackAkkaRpcSystemLoader.java
@@ -19,6 +19,7 @@ package org.apache.flink.runtime.rpc.akka;
import org.apache.flink.configuration.Configuration;
import org.apache.flink.core.classloading.SubmoduleClassLoader;
+import org.apache.flink.runtime.rpc.CleanupOnCloseRpcSystem;
import org.apache.flink.runtime.rpc.RpcSystem;
import org.apache.flink.runtime.rpc.RpcSystemLoader;
import org.apache.flink.runtime.rpc.exceptions.RpcLoaderException;
diff --git
a/flink-rpc/flink-rpc-akka/src/main/java/org/apache/flink/runtime/concurrent/akka/ActorSystemScheduledExecutorAdapter.java
b/flink-rpc/flink-rpc-akka/src/main/java/org/apache/flink/runtime/concurrent/akka/ActorSystemScheduledExecutorAdapter.java
index 35147047103..ff6be5c606f 100644
---
a/flink-rpc/flink-rpc-akka/src/main/java/org/apache/flink/runtime/concurrent/akka/ActorSystemScheduledExecutorAdapter.java
+++
b/flink-rpc/flink-rpc-akka/src/main/java/org/apache/flink/runtime/concurrent/akka/ActorSystemScheduledExecutorAdapter.java
@@ -18,6 +18,7 @@
package org.apache.flink.runtime.concurrent.akka;
+import org.apache.flink.runtime.concurrent.ClassLoadingUtils;
import org.apache.flink.util.Preconditions;
import org.apache.flink.util.concurrent.ScheduledExecutor;
diff --git
a/flink-rpc/flink-rpc-akka/src/main/java/org/apache/flink/runtime/rpc/akka/AkkaInvocationHandler.java
b/flink-rpc/flink-rpc-akka/src/main/java/org/apache/flink/runtime/rpc/akka/AkkaInvocationHandler.java
index a510f2d9409..3cd3c03c134 100644
---
a/flink-rpc/flink-rpc-akka/src/main/java/org/apache/flink/runtime/rpc/akka/AkkaInvocationHandler.java
+++
b/flink-rpc/flink-rpc-akka/src/main/java/org/apache/flink/runtime/rpc/akka/AkkaInvocationHandler.java
@@ -58,7 +58,7 @@ import java.util.concurrent.ExecutionException;
import java.util.concurrent.TimeUnit;
import java.util.concurrent.TimeoutException;
-import static
org.apache.flink.runtime.concurrent.akka.ClassLoadingUtils.guardCompletionWithContextClassLoader;
+import static
org.apache.flink.runtime.concurrent.ClassLoadingUtils.guardCompletionWithContextClassLoader;
import static org.apache.flink.util.Preconditions.checkArgument;
import static org.apache.flink.util.Preconditions.checkNotNull;
diff --git
a/flink-rpc/flink-rpc-akka/src/main/java/org/apache/flink/runtime/rpc/akka/AkkaRpcActor.java
b/flink-rpc/flink-rpc-akka/src/main/java/org/apache/flink/runtime/rpc/akka/AkkaRpcActor.java
index 8029956bbd9..d51f802e5a2 100644
---
a/flink-rpc/flink-rpc-akka/src/main/java/org/apache/flink/runtime/rpc/akka/AkkaRpcActor.java
+++
b/flink-rpc/flink-rpc-akka/src/main/java/org/apache/flink/runtime/rpc/akka/AkkaRpcActor.java
@@ -60,7 +60,7 @@ import java.util.concurrent.atomic.AtomicBoolean;
import scala.concurrent.duration.FiniteDuration;
import scala.concurrent.impl.Promise;
-import static
org.apache.flink.runtime.concurrent.akka.ClassLoadingUtils.runWithContextClassLoader;
+import static
org.apache.flink.runtime.concurrent.ClassLoadingUtils.runWithContextClassLoader;
import static org.apache.flink.util.Preconditions.checkArgument;
import static org.apache.flink.util.Preconditions.checkNotNull;
diff --git
a/flink-rpc/flink-rpc-akka/src/main/java/org/apache/flink/runtime/rpc/akka/AkkaRpcService.java
b/flink-rpc/flink-rpc-akka/src/main/java/org/apache/flink/runtime/rpc/akka/AkkaRpcService.java
index ca2ebc4386d..90a4dd5b787 100644
---
a/flink-rpc/flink-rpc-akka/src/main/java/org/apache/flink/runtime/rpc/akka/AkkaRpcService.java
+++
b/flink-rpc/flink-rpc-akka/src/main/java/org/apache/flink/runtime/rpc/akka/AkkaRpcService.java
@@ -73,9 +73,9 @@ import java.util.function.Function;
import scala.Option;
import scala.reflect.ClassTag$;
-import static
org.apache.flink.runtime.concurrent.akka.ClassLoadingUtils.guardCompletionWithContextClassLoader;
-import static
org.apache.flink.runtime.concurrent.akka.ClassLoadingUtils.runWithContextClassLoader;
-import static
org.apache.flink.runtime.concurrent.akka.ClassLoadingUtils.withContextClassLoader;
+import static
org.apache.flink.runtime.concurrent.ClassLoadingUtils.guardCompletionWithContextClassLoader;
+import static
org.apache.flink.runtime.concurrent.ClassLoadingUtils.runWithContextClassLoader;
+import static
org.apache.flink.runtime.concurrent.ClassLoadingUtils.withContextClassLoader;
import static org.apache.flink.util.Preconditions.checkNotNull;
import static org.apache.flink.util.Preconditions.checkState;
diff --git
a/flink-rpc/flink-rpc-akka/src/test/java/org/apache/flink/runtime/rpc/akka/ContextClassLoadingSettingTest.java
b/flink-rpc/flink-rpc-akka/src/test/java/org/apache/flink/runtime/rpc/akka/ContextClassLoadingSettingTest.java
index f5a055d005d..911726de4ab 100644
---
a/flink-rpc/flink-rpc-akka/src/test/java/org/apache/flink/runtime/rpc/akka/ContextClassLoadingSettingTest.java
+++
b/flink-rpc/flink-rpc-akka/src/test/java/org/apache/flink/runtime/rpc/akka/ContextClassLoadingSettingTest.java
@@ -48,7 +48,7 @@ import java.util.concurrent.ExecutionException;
import java.util.concurrent.TimeUnit;
import java.util.function.Consumer;
-import static
org.apache.flink.runtime.concurrent.akka.ClassLoadingUtils.runWithContextClassLoader;
+import static
org.apache.flink.runtime.concurrent.ClassLoadingUtils.runWithContextClassLoader;
import static org.assertj.core.api.Assertions.assertThat;
import static org.junit.jupiter.api.Assertions.assertEquals;
diff --git
a/flink-rpc/flink-rpc-akka/src/main/java/org/apache/flink/runtime/concurrent/akka/ClassLoadingUtils.java
b/flink-rpc/flink-rpc-core/src/main/java/org/apache/flink/runtime/concurrent/ClassLoadingUtils.java
similarity index 98%
rename from
flink-rpc/flink-rpc-akka/src/main/java/org/apache/flink/runtime/concurrent/akka/ClassLoadingUtils.java
rename to
flink-rpc/flink-rpc-core/src/main/java/org/apache/flink/runtime/concurrent/ClassLoadingUtils.java
index 623c99f89ea..1f617de3b65 100644
---
a/flink-rpc/flink-rpc-akka/src/main/java/org/apache/flink/runtime/concurrent/akka/ClassLoadingUtils.java
+++
b/flink-rpc/flink-rpc-core/src/main/java/org/apache/flink/runtime/concurrent/ClassLoadingUtils.java
@@ -15,7 +15,7 @@
* limitations under the License.
*/
-package org.apache.flink.runtime.concurrent.akka;
+package org.apache.flink.runtime.concurrent;
import org.apache.flink.util.TemporaryClassLoaderContext;
import org.apache.flink.util.concurrent.FutureUtils;
diff --git
a/flink-rpc/flink-rpc-akka-loader/src/main/java/org/apache/flink/runtime/rpc/akka/CleanupOnCloseRpcSystem.java
b/flink-rpc/flink-rpc-core/src/main/java/org/apache/flink/runtime/rpc/CleanupOnCloseRpcSystem.java
similarity index 94%
rename from
flink-rpc/flink-rpc-akka-loader/src/main/java/org/apache/flink/runtime/rpc/akka/CleanupOnCloseRpcSystem.java
rename to
flink-rpc/flink-rpc-core/src/main/java/org/apache/flink/runtime/rpc/CleanupOnCloseRpcSystem.java
index e45f4b5a035..100085def2a 100644
---
a/flink-rpc/flink-rpc-akka-loader/src/main/java/org/apache/flink/runtime/rpc/akka/CleanupOnCloseRpcSystem.java
+++
b/flink-rpc/flink-rpc-core/src/main/java/org/apache/flink/runtime/rpc/CleanupOnCloseRpcSystem.java
@@ -15,12 +15,10 @@
* limitations under the License.
*/
-package org.apache.flink.runtime.rpc.akka;
+package org.apache.flink.runtime.rpc;
import org.apache.flink.configuration.Configuration;
import org.apache.flink.core.classloading.SubmoduleClassLoader;
-import org.apache.flink.runtime.rpc.AddressResolution;
-import org.apache.flink.runtime.rpc.RpcSystem;
import org.apache.flink.util.FileUtils;
import org.apache.flink.util.Preconditions;
@@ -34,7 +32,7 @@ import java.net.UnknownHostException;
import java.nio.file.Path;
/** An {@link RpcSystem} wrapper that cleans up resources after the RPC system
has been closed. */
-class CleanupOnCloseRpcSystem implements RpcSystem {
+public class CleanupOnCloseRpcSystem implements RpcSystem {
private static final Logger LOG =
LoggerFactory.getLogger(CleanupOnCloseRpcSystem.class);
private final RpcSystem rpcSystem;
diff --git
a/flink-rpc/flink-rpc-akka/src/test/java/org/apache/flink/runtime/concurrent/akka/ClassLoadingUtilsTest.java
b/flink-rpc/flink-rpc-core/src/test/java/org/apache/flink/runtime/concurrent/ClassLoadingUtilsTest.java
similarity index 98%
rename from
flink-rpc/flink-rpc-akka/src/test/java/org/apache/flink/runtime/concurrent/akka/ClassLoadingUtilsTest.java
rename to
flink-rpc/flink-rpc-core/src/test/java/org/apache/flink/runtime/concurrent/ClassLoadingUtilsTest.java
index 8424d313ec6..92bf896ae78 100644
---
a/flink-rpc/flink-rpc-akka/src/test/java/org/apache/flink/runtime/concurrent/akka/ClassLoadingUtilsTest.java
+++
b/flink-rpc/flink-rpc-core/src/test/java/org/apache/flink/runtime/concurrent/ClassLoadingUtilsTest.java
@@ -15,7 +15,7 @@
* limitations under the License.
*/
-package org.apache.flink.runtime.concurrent.akka;
+package org.apache.flink.runtime.concurrent;
import org.apache.flink.util.concurrent.Executors;
import org.apache.flink.util.function.SupplierWithException;