[
https://issues.apache.org/jira/browse/BEAM-6661?focusedWorklogId=427747&page=com.atlassian.jira.plugin.system.issuetabpanels:worklog-tabpanel#worklog-427747
]
ASF GitHub Bot logged work on BEAM-6661:
----------------------------------------
Author: ASF GitHub Bot
Created on: 27/Apr/20 18:01
Start Date: 27/Apr/20 18:01
Worklog Time Spent: 10m
Work Description: ibzib commented on a change in pull request #11537:
URL: https://github.com/apache/beam/pull/11537#discussion_r416024189
##########
File path:
runners/java-fn-execution/src/main/java/org/apache/beam/runners/fnexecution/environment/ExternalEnvironmentFactory.java
##########
@@ -145,17 +149,23 @@ public InstructionRequestHandler
getInstructionRequestHandler() {
@Override
public void close() throws Exception {
- finalInstructionHandler.close();
- BeamFnApi.StopWorkerRequest stopWorkerRequest =
-
BeamFnApi.StopWorkerRequest.newBuilder().setWorkerId(workerId).build();
- LOG.debug("Closing worker ID {}", workerId);
- BeamFnApi.StopWorkerResponse stopWorkerResponse =
- BeamFnExternalWorkerPoolGrpc.newBlockingStub(
- ManagedChannelFactory.createDefault()
- .forDescriptor(externalPayload.getEndpoint()))
- .stopWorker(stopWorkerRequest);
- if (!stopWorkerResponse.getError().isEmpty()) {
- throw new RuntimeException(stopWorkerResponse.getError());
+ try {
+ finalInstructionHandler.close();
+ BeamFnApi.StopWorkerRequest stopWorkerRequest =
+
BeamFnApi.StopWorkerRequest.newBuilder().setWorkerId(workerId).build();
+ LOG.debug("Closing worker ID {}", workerId);
+ BeamFnApi.StopWorkerResponse stopWorkerResponse =
+ BeamFnExternalWorkerPoolGrpc.newBlockingStub(managedChannel)
+ .stopWorker(stopWorkerRequest);
+ if (!stopWorkerResponse.getError().isEmpty()) {
+ throw new RuntimeException(stopWorkerResponse.getError());
+ }
+ } finally {
+ managedChannel.shutdown();
+ managedChannel.awaitTermination(10, TimeUnit.SECONDS);
Review comment:
Nit: awaitTermination already returns whether the channel is terminated,
so no need to call isTerminated
##########
File path:
runners/java-fn-execution/src/main/java/org/apache/beam/runners/fnexecution/environment/ExternalEnvironmentFactory.java
##########
@@ -145,17 +149,23 @@ public InstructionRequestHandler
getInstructionRequestHandler() {
@Override
public void close() throws Exception {
- finalInstructionHandler.close();
- BeamFnApi.StopWorkerRequest stopWorkerRequest =
-
BeamFnApi.StopWorkerRequest.newBuilder().setWorkerId(workerId).build();
- LOG.debug("Closing worker ID {}", workerId);
- BeamFnApi.StopWorkerResponse stopWorkerResponse =
- BeamFnExternalWorkerPoolGrpc.newBlockingStub(
- ManagedChannelFactory.createDefault()
- .forDescriptor(externalPayload.getEndpoint()))
- .stopWorker(stopWorkerRequest);
- if (!stopWorkerResponse.getError().isEmpty()) {
- throw new RuntimeException(stopWorkerResponse.getError());
+ try {
+ finalInstructionHandler.close();
+ BeamFnApi.StopWorkerRequest stopWorkerRequest =
+
BeamFnApi.StopWorkerRequest.newBuilder().setWorkerId(workerId).build();
+ LOG.debug("Closing worker ID {}", workerId);
+ BeamFnApi.StopWorkerResponse stopWorkerResponse =
+ BeamFnExternalWorkerPoolGrpc.newBlockingStub(managedChannel)
+ .stopWorker(stopWorkerRequest);
+ if (!stopWorkerResponse.getError().isEmpty()) {
+ throw new RuntimeException(stopWorkerResponse.getError());
+ }
+ } finally {
+ managedChannel.shutdown();
+ managedChannel.awaitTermination(10, TimeUnit.SECONDS);
Review comment:
Why 10 seconds? Does shutdown usually take that long?
----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to 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: 427747)
Time Spent: 0.5h (was: 20m)
> FnApi gRPC setup/teardown glitch
> --------------------------------
>
> Key: BEAM-6661
> URL: https://issues.apache.org/jira/browse/BEAM-6661
> Project: Beam
> Issue Type: Bug
> Components: java-fn-execution
> Affects Versions: 2.11.0
> Reporter: Heejong Lee
> Assignee: Heejong Lee
> Priority: Major
> Time Spent: 0.5h
> Remaining Estimate: 0h
>
> Multiple exceptions are observed during FnApi gRPC setup/teardown. The
> examples are
> {noformat}
> 14:53:22 [grpc-default-executor-1] WARN
> org.apache.beam.runners.fnexecution.logging.GrpcLoggingService - Logging
> client failed unexpectedly.
> 14:53:22 org.apache.beam.vendor.grpc.v1p13p1.io.grpc.StatusRuntimeException:
> CANCELLED: cancelled before receiving half close
> 14:53:22 at
> org.apache.beam.vendor.grpc.v1p13p1.io.grpc.Status.asRuntimeException(Status.java:517)
> 14:53:22 at{noformat}
> {noformat}
>
> 14:52:56 [main] INFO org.apache.beam.runners.flink.FlinkJobServerDriver -
> JobService started on localhost:58179
> 14:52:57 [grpc-default-executor-0] ERROR
> org.apache.beam.runners.fnexecution.jobsubmission.InMemoryJobService -
> Encountered Unexpected Exception for Invocation
> job_bfb7df0e-408e-4bfd-bb3c-432e946ca819
> 14:52:57 org.apache.beam.vendor.grpc.v1p13p1.io.grpc.StatusException:
> NOT_FOUND
> 14:52:57 at
> org.apache.beam.vendor.grpc.v1p13p1.io.grpc.Status.asException(Status.java:534)
> 14:52:57 at
> org.apache.beam.runners.fnexecution.jobsubmission.InMemoryJobService.getInvocation(InMemoryJobService.java:341)
> 14:52:57 at
> org.apache.beam.runners.fnexecution.jobsubmission.InMemoryJobService.getStateStream(InMemoryJobService.java:262)
> 14:52:57 at
> org.apache.beam.model.jobmanagement.v1.JobServiceGrpc$MethodHandlers.invoke(JobServiceGrpc.java:693)
> 14:52:57 at
> org.apache.beam.vendor.grpc.v1p13p1.io.grpc.stub.ServerCalls$UnaryServerCallHandler$UnaryServerCallListener.onHalfClose(ServerCalls.java:171)
> 14:52:57 at
> org.apache.beam.vendor.grpc.v1p13p1.io.grpc.PartialForwardingServerCallListener.onHalfClose(PartialForwardingServerCallListener.java:35)
> 14:52:57 at
> org.apache.beam.vendor.grpc.v1p13p1.io.grpc.ForwardingServerCallListener.onHalfClose(ForwardingServerCallListener.java:23)
> 14:52:57 at
> org.apache.beam.vendor.grpc.v1p13p1.io.grpc.ForwardingServerCallListener$SimpleForwardingServerCallListener.onHalfClose(ForwardingServerCallListener.java:40)
> 14:52:57 at
> org.apache.beam.vendor.grpc.v1p13p1.io.grpc.Contexts$ContextualizedServerCallListener.onHalfClose(Contexts.java:86)
> 14:52:57 at
> org.apache.beam.vendor.grpc.v1p13p1.io.grpc.internal.ServerCallImpl$ServerStreamListenerImpl.halfClosed(ServerCallImpl.java:283)
> 14:52:57 at
> org.apache.beam.vendor.grpc.v1p13p1.io.grpc.internal.ServerImpl$JumpToApplicationThreadServerStreamListener$1HalfClosed.runInContext(ServerImpl.java:707)
> 14:52:57 at
> org.apache.beam.vendor.grpc.v1p13p1.io.grpc.internal.ContextRunnable.run(ContextRunnable.java:37)
> 14:52:57 at
> org.apache.beam.vendor.grpc.v1p13p1.io.grpc.internal.SerializingExecutor.run(SerializingExecutor.java:123)
> 14:52:57 at
> java.util.concurrent.ThreadPoolExecutor.runWorker(ThreadPoolExecutor.java:1149)
> 14:52:57 at
> java.util.concurrent.ThreadPoolExecutor$Worker.run(ThreadPoolExecutor.java:624)
> {noformat}
> {noformat}
> 14:54:50 Feb 07, 2019 10:54:50 PM
> org.apache.beam.vendor.grpc.v1p13p1.io.grpc.internal.ManagedChannelOrphanWrapper$ManagedChannelReference
> cleanQueue
> 14:54:50 SEVERE: *~*~*~ Channel ManagedChannelImpl{logId=628,
> target=localhost:41409} was not shutdown properly!!! ~*~*~*
> 14:54:50 Make sure to call shutdown()/shutdownNow() and wait until
> awaitTermination() returns true.
> 14:54:50 java.lang.RuntimeException: ManagedChannel allocation site
> 14:54:50 at
> org.apache.beam.vendor.grpc.v1p13p1.io.grpc.internal.ManagedChannelOrphanWrapper$ManagedChannelReference.<init>(ManagedChannelOrphanWrapper.java:103)
> 14:54:50 at
> org.apache.beam.vendor.grpc.v1p13p1.io.grpc.internal.ManagedChannelOrphanWrapper.<init>(ManagedChannelOrphanWrapper.java:53)
> 14:54:50 at
> org.apache.beam.vendor.grpc.v1p13p1.io.grpc.internal.ManagedChannelOrphanWrapper.<init>(ManagedChannelOrphanWrapper.java:44)
> 14:54:50 at
> org.apache.beam.vendor.grpc.v1p13p1.io.grpc.internal.AbstractManagedChannelImplBuilder.build(AbstractManagedChannelImplBuilder.java:410)
> 14:54:50 at
> org.apache.beam.sdk.fn.channel.ManagedChannelFactory.forDescriptor(ManagedChannelFactory.java:44)
> 14:54:50 at
> org.apache.beam.runners.fnexecution.environment.ExternalEnvironmentFactory.createEnvironment(ExternalEnvironmentFactory.java:108)
> 14:54:50 at
> org.apache.beam.runners.fnexecution.control.DefaultJobBundleFactory$1.load(DefaultJobBundleFactory.java:154)
> 14:54:50 at
> org.apache.beam.runners.fnexecution.control.DefaultJobBundleFactory$1.load(DefaultJobBundleFactory.java:137)
> 14:54:50 at
> org.apache.beam.vendor.guava.v20_0.com.google.common.cache.LocalCache$LoadingValueReference.loadFuture(LocalCache.java:3628)
> 14:54:50 at
> org.apache.beam.vendor.guava.v20_0.com.google.common.cache.LocalCache$Segment.loadSync(LocalCache.java:2336)
> 14:54:50 at
> org.apache.beam.vendor.guava.v20_0.com.google.common.cache.LocalCache$Segment.lockedGetOrLoad(LocalCache.java:2295)
> 14:54:50 at
> org.apache.beam.vendor.guava.v20_0.com.google.common.cache.LocalCache$Segment.get(LocalCache.java:2208)
> 14:54:50 at
> org.apache.beam.vendor.guava.v20_0.com.google.common.cache.LocalCache.get(LocalCache.java:4053)
> 14:54:50 at
> org.apache.beam.vendor.guava.v20_0.com.google.common.cache.LocalCache.getOrLoad(LocalCache.java:4057)
> 14:54:50 at
> org.apache.beam.vendor.guava.v20_0.com.google.common.cache.LocalCache$LocalLoadingCache.get(LocalCache.java:4986)
> 14:54:50 at
> org.apache.beam.vendor.guava.v20_0.com.google.common.cache.LocalCache$LocalLoadingCache.getUnchecked(LocalCache.java:4992)
> 14:54:50 at
> org.apache.beam.runners.fnexecution.control.DefaultJobBundleFactory.forStage(DefaultJobBundleFactory.java:162){noformat}
--
This message was sent by Atlassian Jira
(v8.3.4#803005)