Skip to content

Commit

Permalink
Fix.
Browse files Browse the repository at this point in the history
  • Loading branch information
ueshin committed Sep 25, 2018
1 parent 52584d9 commit ac0243a
Show file tree
Hide file tree
Showing 2 changed files with 15 additions and 7 deletions.
4 changes: 2 additions & 2 deletions python/pyspark/sql/context.py
Expand Up @@ -485,8 +485,8 @@ def __init__(self, sparkContext, jhiveContext=None):
"SparkSession.builder.enableHiveSupport().getOrCreate() instead.",
DeprecationWarning)
if jhiveContext is None:
sparkSession = SparkSession.builder.getOrCreate()
sparkSession.sparkContext._conf.set("spark.sql.catalogImplementation", "hive")
sparkContext._conf.set("spark.sql.catalogImplementation", "hive")
sparkSession = SparkSession.builder._sparkContext(sparkContext).getOrCreate()
else:
sparkSession = SparkSession(sparkContext, jhiveContext.sparkSession())
SQLContext.__init__(self, sparkContext, sparkSession, jhiveContext)
Expand Down
18 changes: 13 additions & 5 deletions python/pyspark/sql/session.py
Expand Up @@ -83,6 +83,7 @@ class Builder(object):

_lock = RLock()
_options = {}
_sc = None

@since(2.0)
def config(self, key=None, value=None, conf=None):
Expand Down Expand Up @@ -139,6 +140,10 @@ def enableHiveSupport(self):
"""
return self.config("spark.sql.catalogImplementation", "hive")

def _sparkContext(self, sc):
self._sc = sc
return self

@since(2.0)
def getOrCreate(self):
"""Gets an existing :class:`SparkSession` or, if there is no existing one, creates a
Expand Down Expand Up @@ -167,11 +172,14 @@ def getOrCreate(self):
from pyspark.conf import SparkConf
session = SparkSession._instantiatedSession
if session is None or session._sc._jsc is None:
sparkConf = SparkConf()
for key, value in self._options.items():
sparkConf.set(key, value)
sc = SparkContext.getOrCreate(sparkConf)
# This SparkContext may be an existing one.
if self._sc is not None:
sc = self._sc
else:
sparkConf = SparkConf()
for key, value in self._options.items():
sparkConf.set(key, value)
sc = SparkContext.getOrCreate(sparkConf)
# This SparkContext may be an existing one.
# Do not update `SparkConf` for existing `SparkContext`, as it's shared
# by all sessions.
session = SparkSession(sc)
Expand Down

0 comments on commit ac0243a

Please sign in to comment.