Skip to content

Commit 56903b8

Browse files
committed
remove unnecessary keyword
1 parent 30abf51 commit 56903b8

File tree

3 files changed

+7
-27
lines changed

3 files changed

+7
-27
lines changed

python/pyspark/sql/connect/client/core.py

Lines changed: 2 additions & 16 deletions
Original file line numberDiff line numberDiff line change
@@ -709,8 +709,8 @@ def __init__(
709709
# cleanup ml cache if possible
710710
atexit.register(self._cleanup_ml_cache)
711711

712-
self.global_user_context_extensions = [] # EDGE
713-
self.global_user_context_extensions_lock = threading.Lock() # EDGE
712+
self.global_user_context_extensions = []
713+
self.global_user_context_extensions_lock = threading.Lock()
714714

715715
@property
716716
def _stub(self) -> grpc_lib.SparkConnectServiceStub:
@@ -1243,7 +1243,6 @@ def token(self) -> Optional[str]:
12431243
"""
12441244
return self._builder.token
12451245

1246-
# BEGIN-EDGE
12471246
def _update_request_with_user_context_extensions(
12481247
self,
12491248
req: Union[
@@ -1262,8 +1261,6 @@ def _update_request_with_user_context_extensions(
12621261
for _, extension in self.thread_local.user_context_extensions:
12631262
req.user_context.extensions.append(extension)
12641263

1265-
# END-EDGE
1266-
12671264
def _execute_plan_request_with_metadata(
12681265
self, operation_id: Optional[str] = None
12691266
) -> pb2.ExecutePlanRequest:
@@ -1294,9 +1291,7 @@ def _execute_plan_request_with_metadata(
12941291
messageParameters={"arg_name": "operation_id", "origin": str(ve)},
12951292
)
12961293
req.operation_id = operation_id
1297-
# BEGIN-EDGE
12981294
self._update_request_with_user_context_extensions(req)
1299-
# END-EDGE
13001295
return req
13011296

13021297
def _analyze_plan_request_with_metadata(self) -> pb2.AnalyzePlanRequest:
@@ -1307,9 +1302,7 @@ def _analyze_plan_request_with_metadata(self) -> pb2.AnalyzePlanRequest:
13071302
req.client_type = self._builder.userAgent
13081303
if self._user_id:
13091304
req.user_context.user_id = self._user_id
1310-
# BEGIN-EDGE
13111305
self._update_request_with_user_context_extensions(req)
1312-
# END-EDGE
13131306
return req
13141307

13151308
def _analyze(self, method: str, **kwargs: Any) -> AnalyzeResult:
@@ -1724,9 +1717,7 @@ def _config_request_with_metadata(self) -> pb2.ConfigRequest:
17241717
req.client_type = self._builder.userAgent
17251718
if self._user_id:
17261719
req.user_context.user_id = self._user_id
1727-
# BEGIN-EDGE
17281720
self._update_request_with_user_context_extensions(req)
1729-
# END-EDGE
17301721
return req
17311722

17321723
def get_configs(self, *keys: str) -> Tuple[Optional[str], ...]:
@@ -1902,7 +1893,6 @@ def _throw_if_invalid_tag(self, tag: str) -> None:
19021893
messageParameters={"arg_name": "Spark Connect tag", "arg_value": tag},
19031894
)
19041895

1905-
# BEGIN-EDGE
19061896
def add_threadlocal_user_context_extension(self, extension: any_pb2.Any) -> str:
19071897
if not hasattr(self.thread_local, "user_context_extensions"):
19081898
self.thread_local.user_context_extensions = list()
@@ -1935,8 +1925,6 @@ def clear_user_context_extensions(self) -> None:
19351925
with self.global_user_context_extensions_lock:
19361926
self.global_user_context_extensions = list()
19371927

1938-
# END-EDGE
1939-
19401928
def _handle_error(self, error: Exception) -> NoReturn:
19411929
"""
19421930
Handle errors that occur during RPC calls.
@@ -1977,9 +1965,7 @@ def _fetch_enriched_error(self, info: "ErrorInfo") -> Optional[pb2.FetchErrorDet
19771965
req.client_observed_server_side_session_id = self._server_session_id
19781966
if self._user_id:
19791967
req.user_context.user_id = self._user_id
1980-
# BEGIN-EDGE
19811968
self._update_request_with_user_context_extensions(req)
1982-
# END-EDGE
19831969
try:
19841970
return self._stub.FetchErrorDetails(req, metadata=self._builder.metadata())
19851971
except grpc.RpcError:

python/pyspark/sql/connect/session.py

Lines changed: 1 addition & 4 deletions
Original file line numberDiff line numberDiff line change
@@ -42,7 +42,7 @@
4242
ClassVar,
4343
)
4444

45-
import google.protobuf.any_pb2 as any_pb2 # EDGE
45+
import google.protobuf.any_pb2 as any_pb2
4646
import numpy as np
4747
import pandas as pd
4848
import pyarrow as pa
@@ -895,7 +895,6 @@ def clearTags(self) -> None:
895895

896896
clearTags.__doc__ = PySparkSession.clearTags.__doc__
897897

898-
# BEGIN-EDGE
899898
def addThreadlocalUserContextExtension(self, extension: any_pb2.Any) -> str:
900899
"""
901900
Add a user context extension to the current session in the current thread.
@@ -950,8 +949,6 @@ def clearUserContextExtensions(self) -> None:
950949
"""
951950
self.client.clear_user_context_extensions()
952951

953-
# END-EDGE
954-
955952
def stop(self) -> None:
956953
"""
957954
Release the current session and close the GRPC connection to the Spark Connect server.

python/pyspark/sql/tests/connect/client/test_client.py

Lines changed: 4 additions & 7 deletions
Original file line numberDiff line numberDiff line change
@@ -136,11 +136,11 @@ class MockService:
136136
def __init__(self, session_id: str):
137137
self._session_id = session_id
138138
self.req = None
139-
self.client_user_context_extensions = [] # EDGE
139+
self.client_user_context_extensions = []
140140

141141
def ExecutePlan(self, req: proto.ExecutePlanRequest, metadata):
142142
self.req = req
143-
self.client_user_context_extensions = req.user_context.extensions # EDGE
143+
self.client_user_context_extensions = req.user_context.extensions
144144
resp = proto.ExecutePlanResponse()
145145
resp.session_id = self._session_id
146146
resp.operation_id = req.operation_id
@@ -161,14 +161,14 @@ def ExecutePlan(self, req: proto.ExecutePlanRequest, metadata):
161161

162162
def Interrupt(self, req: proto.InterruptRequest, metadata):
163163
self.req = req
164-
self.client_user_context_extensions = req.user_context.extensions # EDGE
164+
self.client_user_context_extensions = req.user_context.extensions
165165
resp = proto.InterruptResponse()
166166
resp.session_id = self._session_id
167167
return resp
168168

169169
def Config(self, req: proto.ConfigRequest, metadata):
170170
self.req = req
171-
self.client_user_context_extensions = req.user_context.extensions # EDGE
171+
self.client_user_context_extensions = req.user_context.extensions
172172
resp = proto.ConfigResponse()
173173
resp.session_id = self._session_id
174174
if req.operation.HasField("get"):
@@ -233,7 +233,6 @@ def userId(self) -> Optional[str]:
233233

234234
self.assertEqual(client._user_id, "abc")
235235

236-
# BEGIN-EDGE
237236
def test_user_context_extension(self):
238237
client = SparkConnectClient("sc://foo/", use_reattachable_execute=False)
239238
mock = MockService(client._session_id)
@@ -321,8 +320,6 @@ def test_user_context_extension(self):
321320
self.assertFalse(exlocal2 in mock.client_user_context_extensions)
322321
self.assertFalse(exglobal2 in mock.client_user_context_extensions)
323322

324-
# END-EDGE
325-
326323
def test_interrupt_all(self):
327324
client = SparkConnectClient("sc://foo/;token=bar", use_reattachable_execute=False)
328325
mock = MockService(client._session_id)

0 commit comments

Comments
 (0)