This is an automated email from the ASF dual-hosted git repository.

xxyu pushed a commit to branch kylin-on-parquet-v2
in repository https://gitbox.apache.org/repos/asf/kylin.git

commit 591c0e61e5cdd74b2bce2d36d5b3995130673559
Author: feng.zhu <fish...@outlook.com>
AuthorDate: Fri Jan 29 12:07:01 2021 +0800

    KYLIN-4889, fix spark engine in local mode
---
 .../scala/org/apache/spark/sql/KylinSession.scala  |  3 +++
 .../org/apache/spark/sql/SparderContext.scala      | 31 +++++++++-------------
 2 files changed, 16 insertions(+), 18 deletions(-)

diff --git 
a/kylin-spark-project/kylin-spark-query/src/main/scala/org/apache/spark/sql/KylinSession.scala
 
b/kylin-spark-project/kylin-spark-query/src/main/scala/org/apache/spark/sql/KylinSession.scala
index e3c532e..892cf9c 100644
--- 
a/kylin-spark-project/kylin-spark-query/src/main/scala/org/apache/spark/sql/KylinSession.scala
+++ 
b/kylin-spark-project/kylin-spark-query/src/main/scala/org/apache/spark/sql/KylinSession.scala
@@ -199,6 +199,9 @@ object KylinSession extends Logging {
         }
         sparkConf.set("spark.yarn.am.extraJavaOptions",
           s"$yarnAMJavaOptions $amKerberosConf")
+      } else {
+        // in case spark conf is overridden by kylinconfig
+        sparkConf.setMaster("local")
       }
 
       sparkConf
diff --git 
a/kylin-spark-project/kylin-spark-query/src/main/scala/org/apache/spark/sql/SparderContext.scala
 
b/kylin-spark-project/kylin-spark-query/src/main/scala/org/apache/spark/sql/SparderContext.scala
index 0d25dba..d4d6f49 100644
--- 
a/kylin-spark-project/kylin-spark-query/src/main/scala/org/apache/spark/sql/SparderContext.scala
+++ 
b/kylin-spark-project/kylin-spark-query/src/main/scala/org/apache/spark/sql/SparderContext.scala
@@ -131,26 +131,21 @@ object SparderContext extends Logging {
           override def run(): Unit = {
             try {
               val kylinConf: KylinConfig = KylinConfig.getInstanceFromEnv
-              val sparkSession = System.getProperty("spark.local") match {
+              val master = System.getProperty("spark.local") match {
                 case "true" =>
-                  SparkSession.builder
-                    .master("local")
-                    .appName(kylinConf.getSparderAppName)
-                    .withExtensions { ext =>
-                      ext.injectPlannerStrategy(_ => KylinSourceStrategy)
-                    }
-                    .enableHiveSupport()
-                    .getOrCreateKylinSession()
+                  "local"
                 case _ =>
-                  SparkSession.builder
-                    .appName(kylinConf.getSparderAppName)
-                    .master("yarn-client")
-                    .withExtensions { ext =>
-                      ext.injectPlannerStrategy(_ => KylinSourceStrategy)
-                    }
-                    .enableHiveSupport()
-                    .getOrCreateKylinSession()
+                  "yarn-client"
               }
+              val sparkSession = SparkSession.builder
+                .master(master)
+                .appName(kylinConf.getSparderAppName)
+                .withExtensions { ext =>
+                  ext.injectPlannerStrategy(_ => KylinSourceStrategy)
+                }
+                .enableHiveSupport()
+                .getOrCreateKylinSession()
+
               if (kylinConf.isKylinMetricsReporterForQueryEnabled) {
                 val appStatusListener = new SparderMetricsListener()
                 sparkSession.sparkContext.addSparkListener(appStatusListener)
@@ -177,7 +172,7 @@ object SparderContext extends Logging {
                   .getContextClassLoader
                   .toString)
               initMonitorEnv()
-              System.getProperty("spark.local") match {
+              master match {
                 case "true" =>
                   master_app_url = "http://localhost:"; + 
sparkSession.sparkContext.getConf
                     .get("spark.ui.port", "4040")

Reply via email to