Skip to content
Closed
Show file tree
Hide file tree
Changes from all commits
Commits
File filter

Filter by extension

Filter by extension

Conversations
Failed to load comments.
Loading
Jump to
Jump to file
Failed to load files.
Loading
Diff view
Diff view
2 changes: 1 addition & 1 deletion python/pyspark/sql/connect/client/core.py
Original file line number Diff line number Diff line change
Expand Up @@ -1108,7 +1108,7 @@ def close(self) -> None:
"""
Close the channel.
"""
ExecutePlanResponseReattachableIterator.shutdown_threadpool()
ExecutePlanResponseReattachableIterator.shutdown()
self._channel.close()
self._closed = True

Expand Down
23 changes: 8 additions & 15 deletions python/pyspark/sql/connect/client/reattach.py
Original file line number Diff line number Diff line change
Expand Up @@ -74,7 +74,7 @@ def _release_thread_pool(cls) -> ThreadPool:
return cls._release_thread_pool_instance

@classmethod
def shutdown_threadpool(cls: Type["ExecutePlanResponseReattachableIterator"]) -> None:
def shutdown(cls: Type["ExecutePlanResponseReattachableIterator"]) -> None:
"""
When the channel is closed, this method will be called before, to make sure all
outstanding calls are closed.
Expand All @@ -85,23 +85,14 @@ def shutdown_threadpool(cls: Type["ExecutePlanResponseReattachableIterator"]) ->
cls._release_thread_pool.join() # type: ignore[attr-defined]
cls._release_thread_pool_instance = None

def shutdown(self: "ExecutePlanResponseReattachableIterator") -> None:
"""
When the channel is closed, this method will be called before, to make sure all
outstanding calls are closed, and mark this iterator is shutdown.
"""
with self._lock:
self.shutdown_threadpool()
self._is_shutdown = True

def __init__(
self,
request: pb2.ExecutePlanRequest,
stub: grpc_lib.SparkConnectServiceStub,
retrying: Callable[[], Retrying],
metadata: Iterable[Tuple[str, str]],
):
self._is_shutdown = False
self._release_thread_pool # Trigger initialization
self._request = request
self._retrying = retrying
if request.operation_id:
Expand Down Expand Up @@ -219,8 +210,9 @@ def target() -> None:
except Exception as e:
warnings.warn(f"ReleaseExecute failed with exception: {e}.")

if not self._is_shutdown:
self._release_thread_pool.apply_async(target)
with self._lock:
if self._release_thread_pool_instance is not None:
self._release_thread_pool.apply_async(target)

def _release_all(self) -> None:
"""
Expand All @@ -243,8 +235,9 @@ def target() -> None:
except Exception as e:
warnings.warn(f"ReleaseExecute failed with exception: {e}.")

if not self._is_shutdown:
self._release_thread_pool.apply_async(target)
with self._lock:
if self._release_thread_pool_instance is not None:
self._release_thread_pool.apply_async(target)
self._result_complete = True

def _call_iter(self, iter_fun: Callable) -> Any:
Expand Down