This is an automated email from the ASF dual-hosted git repository.
xxyu pushed a commit to branch main
in repository https://gitbox.apache.org/repos/asf/kylin.git
The following commit(s) were added to refs/heads/main by this push:
new 5312bac HOTFIX for spark3
5312bac is described below
commit 5312bac727e5d64bfa7be999de0d2bddc273a3b8
Author: Zhichao Zhang <[email protected]>
AuthorDate: Sat Aug 7 16:51:13 2021 +0800
HOTFIX for spark3
---
build/bin/kylin.sh | 13 +-
.../src/main/resources/kylin-defaults.properties | 15 +-
docker/dockerfile/standalone/conf/bin/kylin.sh | 201 +++++++--------------
.../kylin/spark/classloader/SparkClassLoader.java | 1 -
.../engine/spark/metadata/cube/PathManager.java | 2 +-
.../kylin/engine/spark/job/NSparkExecutable.java | 3 -
.../org/apache/spark/conf/rule/SparkConfRule.scala | 16 +-
.../org/apache/spark/utils/ResourceUtils.scala | 7 +-
.../scala/org/apache/spark/sql/KylinSession.scala | 17 +-
pom.xml | 1 +
.../kylin/rest/controller/CubeController.java | 10 +-
11 files changed, 119 insertions(+), 167 deletions(-)
diff --git a/build/bin/kylin.sh b/build/bin/kylin.sh
index 77d84fb..9498f0b 100755
--- a/build/bin/kylin.sh
+++ b/build/bin/kylin.sh
@@ -143,18 +143,12 @@ function retrieveDependency() {
# compose hadoop_dependencies
hadoop_dependencies=${hadoop_dependencies}:`hadoop classpath`
-# if [ -n "${hbase_dependency}" ]; then
-# hadoop_dependencies=${hadoop_dependencies}:${hbase_dependency}
-# fi
if [ -n "${hive_dependency}" ]; then
- hadoop_dependencies=${hadoop_dependencies}:${hive_dependency}
+ hadoop_dependencies=${hive_dependency}:${hadoop_dependencies}
fi
if [ -n "${kafka_dependency}" ]; then
hadoop_dependencies=${hadoop_dependencies}:${kafka_dependency}
fi
- if [ -n "${spark_dependency}" ]; then
- hadoop_dependencies=${hadoop_dependencies}:${spark_dependency}
- fi
# compose KYLIN_TOMCAT_CLASSPATH
tomcat_classpath=${tomcat_root}/bin/bootstrap.jar:${tomcat_root}/bin/tomcat-juli.jar:${tomcat_root}/lib/*
@@ -305,6 +299,11 @@ function retrieveStartCommand() {
verbose "java opts is ${KYLIN_EXTRA_START_OPTS} ${KYLIN_TOMCAT_OPTS}"
verbose "java classpath is ${KYLIN_TOMCAT_CLASSPATH}"
classpathDebug ${KYLIN_TOMCAT_CLASSPATH}
+ # For Sparder
+ if [ -z ${HADOOP_CONF_DIR} ]
+ then
+ export HADOOP_CONF_DIR=${kylin_hadoop_conf_dir}
+ fi
$JAVA ${KYLIN_EXTRA_START_OPTS} ${KYLIN_TOMCAT_OPTS} -classpath
${KYLIN_TOMCAT_CLASSPATH} org.apache.catalina.startup.Bootstrap start >>
${KYLIN_HOME}/logs/kylin.out 2>&1 & echo $! > ${KYLIN_HOME}/pid &
}
diff --git a/core-common/src/main/resources/kylin-defaults.properties
b/core-common/src/main/resources/kylin-defaults.properties
index 3bff28c..c990ac2 100644
--- a/core-common/src/main/resources/kylin-defaults.properties
+++ b/core-common/src/main/resources/kylin-defaults.properties
@@ -18,7 +18,7 @@
### METADATA | ENV ###
# The metadata store has two implementations(RDBMS/HBase), while RDBMS is
recommended in Kylin 4.X
-# Please refer to
https://cwiki.apache.org/confluence/display/KYLIN/How+to+use+HBase+metastore+in+Kylin+4.0
if you prefer HBase
+# Please check https://kylin.apache.org/docs/tutorial/mysql_metastore.html
kylin.metadata.url=kylin_metadata@jdbc,url=jdbc:mysql://localhost:3306/kylin,username=XXXX,password=XXXXXX,maxActive=10,maxIdle=10
# metadata cache sync retry times
@@ -243,15 +243,16 @@ kylin.security.saml.context-path=/kylin
# Spark conf (default is in spark/conf/spark-defaults.conf)
kylin.engine.spark-conf.spark.master=yarn
-#kylin.engine.spark-conf.spark.submit.deployMode=client
+kylin.engine.spark-conf.spark.submit.deployMode=client
kylin.engine.spark-conf.spark.yarn.queue=default
#kylin.engine.spark-conf.spark.executor.cores=1
#kylin.engine.spark-conf.spark.executor.memory=4G
#kylin.engine.spark-conf.spark.executor.instances=1
#kylin.engine.spark-conf.spark.executor.memoryOverhead=1024M
-kylin.engine.spark-conf.spark.driver.cores=1
-kylin.engine.spark-conf.spark.driver.memory=1G
-kylin.engine.spark-conf.spark.shuffle.service.enabled=true
+#kylin.engine.spark-conf.spark.driver.cores=1
+#kylin.engine.spark-conf.spark.driver.memory=2G
+#kylin.engine.spark-conf.spark.driver.memoryOverhead=512M
+#kylin.engine.spark-conf.spark.shuffle.service.enabled=false
kylin.engine.spark-conf.spark.eventLog.enabled=true
kylin.engine.spark-conf.spark.eventLog.dir=hdfs\:///kylin/spark-history
kylin.engine.spark-conf.spark.history.fs.logDirectory=hdfs\:///kylin/spark-history
@@ -266,6 +267,7 @@
kylin.engine.spark-conf.spark.executor.extraJavaOptions=-Dfile.encoding=UTF-8 -D
# uncomment for HDP
#kylin.engine.spark-conf.spark.driver.extraJavaOptions=-Dhdp.version=current
#kylin.engine.spark-conf.spark.yarn.am.extraJavaOptions=-Dhdp.version=current
+kylin.engine.spark-conf.spark.driver.extraJavaOptions=-XX:+CrashOnOutOfMemoryError
### SPARK QUERY ENGINE CONFIGS (a.k.a. Sparder Context) ###
# Enlarge cores and memory to improve query performance in production env,
please check https://cwiki.apache.org/confluence/display/KYLIN/User+Manual+4.X
@@ -282,8 +284,9 @@ kylin.query.spark-conf.spark.executor.memoryOverhead=1G
kylin.query.spark-conf.spark.serializer=org.apache.spark.serializer.JavaSerializer
#kylin.query.spark-conf.spark.sql.shuffle.partitions=40
#kylin.query.spark-conf.spark.yarn.jars=hdfs://localhost:9000/spark2_jars/*
+kylin.query.spark-conf.spark.hadoop.yarn.timeline-service.enabled=false
-kylin.query.spark-conf.spark.executor.extraJavaOptions=-Dhdp.version=current
-Dlog4j.configuration=spark-executor-log4j.properties -Dlog4j.debug
-Dkylin.hdfs.working.dir=${kylin.env.hdfs-working-dir}
-Dkylin.metadata.identifier=${kylin.metadata.url.identifier}
-Dkylin.spark.category=sparder -Dkylin.spark.project=${job.project}
+kylin.query.spark-conf.spark.executor.extraJavaOptions=-Dhdp.version=current
-Dlog4j.configuration=spark-executor-log4j.properties -Dlog4j.debug
-Dkylin.hdfs.working.dir=${kylin.env.hdfs-working-dir}
-Dkylin.metadata.identifier=${kylin.metadata.url.identifier}
-Dkylin.spark.category=sparder
# uncomment for HDP
#kylin.query.spark-conf.spark.driver.extraJavaOptions=-Dhdp.version=current
#kylin.query.spark-conf.spark.yarn.am.extraJavaOptions=-Dhdp.version=current
diff --git a/docker/dockerfile/standalone/conf/bin/kylin.sh
b/docker/dockerfile/standalone/conf/bin/kylin.sh
index a691cec..9498f0b 100755
--- a/docker/dockerfile/standalone/conf/bin/kylin.sh
+++ b/docker/dockerfile/standalone/conf/bin/kylin.sh
@@ -143,20 +143,12 @@ function retrieveDependency() {
# compose hadoop_dependencies
hadoop_dependencies=${hadoop_dependencies}:`hadoop classpath`
-# if [ -n "${hbase_dependency}" ]; then
-# hadoop_dependencies=${hadoop_dependencies}:${hbase_dependency}
-# fi
if [ -n "${hive_dependency}" ]; then
- #hadoop_dependencies=${hadoop_dependencies}:${hive_dependency}
hadoop_dependencies=${hive_dependency}:${hadoop_dependencies}
fi
if [ -n "${kafka_dependency}" ]; then
hadoop_dependencies=${hadoop_dependencies}:${kafka_dependency}
fi
- if [ -n "${spark_dependency}" ]; then
- #hadoop_dependencies=${hadoop_dependencies}:${spark_dependency}
- hadoop_dependencies=${spark_dependency}:${hadoop_dependencies}
- fi
# compose KYLIN_TOMCAT_CLASSPATH
tomcat_classpath=${tomcat_root}/bin/bootstrap.jar:${tomcat_root}/bin/tomcat-juli.jar:${tomcat_root}/lib/*
@@ -271,14 +263,8 @@ function runTool() {
exec $JAVA ${KYLIN_EXTRA_START_OPTS} ${KYLIN_TOOL_OPTS} -classpath
${KYLIN_TOOL_CLASSPATH} "$@"
}
-if [ "$2" == "--reload-dependency" ]
-then
- reload_dependency=1
-fi
-
# start command
-if [ "$1" == "start" ]
-then
+function retrieveStartCommand() {
if [ -f "${KYLIN_HOME}/pid" ]
then
PID=`cat $KYLIN_HOME/pid`
@@ -313,23 +299,15 @@ then
verbose "java opts is ${KYLIN_EXTRA_START_OPTS} ${KYLIN_TOMCAT_OPTS}"
verbose "java classpath is ${KYLIN_TOMCAT_CLASSPATH}"
classpathDebug ${KYLIN_TOMCAT_CLASSPATH}
+ # For Sparder
+ if [ -z ${HADOOP_CONF_DIR} ]
+ then
+ export HADOOP_CONF_DIR=${kylin_hadoop_conf_dir}
+ fi
$JAVA ${KYLIN_EXTRA_START_OPTS} ${KYLIN_TOMCAT_OPTS} -classpath
${KYLIN_TOMCAT_CLASSPATH} org.apache.catalina.startup.Bootstrap start >>
${KYLIN_HOME}/logs/kylin.out 2>&1 & echo $! > ${KYLIN_HOME}/pid &
+}
- echo ""
- echo "A new Kylin instance is started by $USER. To stop it, run 'kylin.sh
stop'"
- echo "Check the log at ${KYLIN_HOME}/logs/kylin.log"
- echo "Web UI is at http://${KYLIN_REST_ADDRESS}/kylin"
- exit 0
-
-# run command
-elif [ "$1" == "run" ]
-then
- retrieveStartCommand
- ${start_command}
-
-# stop command
-elif [ "$1" == "stop" ]
-then
+function retrieveStopCommand() {
if [ -f "${KYLIN_HOME}/pid" ]
then
PID=`cat $KYLIN_HOME/pid`
@@ -342,7 +320,7 @@ then
for ((i=0; i<$LOOP_COUNTER; i++))
do
- # wait to process stopped
+ # wait to process stopped
sleep $WAIT_TIME
if ps -p $PID > /dev/null ; then
echo "Stopping in progress. Will check after $WAIT_TIME
secs again..."
@@ -364,128 +342,71 @@ then
fi
fi
- # process is killed , remove pid file
+ # process is killed , remove pid file
rm -rf ${KYLIN_HOME}/pid
echo "Kylin with pid ${PID} has been stopped."
- exit 0
+ return 0
else
- quit "Kylin with pid ${PID} is not running"
+ echo "Kylin with pid ${PID} is not running"
+ return 1
fi
else
- quit "Kylin is not running"
+ return 1
fi
+}
-# streaming command
-elif [ "$1" == "streaming" ]
+if [ "$2" == "--reload-dependency" ]
then
- if [ $# -lt 2 ]
- then
- echo "Invalid input args $@"
- exit -1
- fi
- if [ "$2" == "start" ]
- then
- if [ -f "${KYLIN_HOME}/streaming_receiver_pid" ]
- then
- PID=`cat $KYLIN_HOME/streaming_receiver_pid`
- if ps -p $PID > /dev/null
- then
- echo "Kylin streaming receiver is running, stop it first"
- exit 1
- fi
- fi
- #retrive $hbase_dependency
- metadataUrl=`${dir}/get-properties.sh kylin.metadata.url`
- if [[ "${metadataUrl##*@}" == "hbase" ]]
- then
- source ${dir}/find-hbase-dependency.sh
- fi
- #retrive $KYLIN_EXTRA_START_OPTS
- if [ -f "${KYLIN_HOME}/conf/setenv.sh" ]
- then source ${KYLIN_HOME}/conf/setenv.sh
- fi
+ reload_dependency=1
+fi
- mkdir -p ${KYLIN_HOME}/ext
- HBASE_CLASSPATH=`hbase classpath`
- #echo "hbase class path:"$HBASE_CLASSPATH
-
STREAM_CLASSPATH=${KYLIN_HOME}/lib/streaming/*:${KYLIN_HOME}/ext/*:${HBASE_CLASSPATH}
-
- # KYLIN_EXTRA_START_OPTS is for customized settings, checkout
bin/setenv.sh
- $JAVA -cp $STREAM_CLASSPATH ${KYLIN_EXTRA_START_OPTS} \
- -Dlog4j.configuration=stream-receiver-log4j.properties\
- -DKYLIN_HOME=${KYLIN_HOME}\
- -Dkylin.hbase.dependency=${hbase_dependency} \
- org.apache.kylin.stream.server.StreamingReceiver $@ >
${KYLIN_HOME}/logs/streaming_receiver.out 2>&1 & echo $! >
${KYLIN_HOME}/streaming_receiver_pid &
- exit 0
- elif [ "$2" == "stop" ]
- then
- if [ ! -f "${KYLIN_HOME}/streaming_receiver_pid" ]
- then
- echo "Streaming receiver is not running, please check"
- exit 1
- fi
- PID=`cat ${KYLIN_HOME}/streaming_receiver_pid`
- if [ "$PID" = "" ]
- then
- echo "Streaming receiver is not running, please check"
- exit 1
- else
- echo "Stopping streaming receiver: $PID"
- WAIT_TIME=2
- LOOP_COUNTER=20
- if ps -p $PID > /dev/null
- then
- kill $PID
-
- for ((i=0; i<$LOOP_COUNTER; i++))
- do
- # wait to process stopped
- sleep $WAIT_TIME
- if ps -p $PID > /dev/null ; then
- echo "Stopping in progress. Will check after
$WAIT_TIME secs again..."
- continue;
- else
- break;
- fi
- done
-
- # if process is still around, use kill -9
- if ps -p $PID > /dev/null
- then
- echo "Initial kill failed, getting serious now..."
- kill -9 $PID
- sleep 1 #give kill -9 sometime to "kill"
- if ps -p $PID > /dev/null
- then
- quit "Warning, even kill -9 failed, giving up! Sorry..."
- fi
- fi
+# start command
+if [ "$1" == "start" ]
+then
+ retrieveStartCommand
- # process is killed , remove pid file
- rm -rf ${KYLIN_HOME}/streaming_receiver_pid
- echo "Kylin streaming receiver with pid ${PID} has been
stopped."
- exit 0
- else
- quit "Kylin streaming receiver with pid ${PID} is not running"
- fi
- fi
- elif [[ "$2" = org.apache.kylin.* ]]
+ echo ""
+ echo "A new Kylin instance is started by $USER. To stop it, run 'kylin.sh
stop'"
+ echo "Check the log at ${KYLIN_HOME}/logs/kylin.log"
+ echo "Web UI is at http://${KYLIN_REST_ADDRESS}/kylin"
+ exit 0
+
+# run command
+elif [ "$1" == "run" ]
+then
+ retrieveStartCommand
+ ${start_command}
+
+# stop command
+elif [ "$1" == "stop" ]
+then
+ retrieveStopCommand
+ if [[ $? == 0 ]]
then
- source ${KYLIN_HOME}/conf/setenv.sh
- HBASE_CLASSPATH=`hbase classpath`
- #echo "hbase class path:"$HBASE_CLASSPATH
-
STREAM_CLASSPATH=${KYLIN_HOME}/lib/streaming/*:${KYLIN_HOME}/ext/*:${HBASE_CLASSPATH}
-
- shift
- # KYLIN_EXTRA_START_OPTS is for customized settings, checkout
bin/setenv.sh
- $JAVA -cp $STREAM_CLASSPATH ${KYLIN_EXTRA_START_OPTS} \
- -Dlog4j.configuration=stream-receiver-log4j.properties\
- -DKYLIN_HOME=${KYLIN_HOME}\
- -Dkylin.hbase.dependency=${hbase_dependency} \
- "$@"
exit 0
+ else
+ quit "Kylin is not running"
fi
+# restart command
+elif [ "$1" == "restart" ]
+then
+ echo "Restarting kylin..."
+ echo "--> Stopping kylin first if it's running..."
+ retrieveStopCommand
+ if [[ $? != 0 ]]
+ then
+ echo "Kylin is not running, now start it"
+ fi
+ echo "--> Start kylin..."
+ retrieveStartCommand
+
+ echo ""
+ echo "A new Kylin instance is started by $USER. To stop it, run 'kylin.sh
stop'"
+ echo "Check the log at ${KYLIN_HOME}/logs/kylin.log"
+ echo "Web UI is at http://${KYLIN_REST_ADDRESS}/kylin"
+ exit 0
+
elif [ "$1" = "version" ]
then
runTool org.apache.kylin.common.KylinVersion
@@ -500,5 +421,5 @@ elif [[ "$1" = org.apache.kylin.* ]]
then
runTool "$@"
else
- quit "Usage: 'kylin.sh [-v] start' or 'kylin.sh [-v] stop'"
+ quit "Usage: 'kylin.sh [-v] start' or 'kylin.sh [-v] stop' or 'kylin.sh
[-v] restart'"
fi
diff --git
a/kylin-spark-project/kylin-spark-classloader/src/main/java/org/apache/kylin/spark/classloader/SparkClassLoader.java
b/kylin-spark-project/kylin-spark-classloader/src/main/java/org/apache/kylin/spark/classloader/SparkClassLoader.java
index d74fb26..2dc0292 100644
---
a/kylin-spark-project/kylin-spark-classloader/src/main/java/org/apache/kylin/spark/classloader/SparkClassLoader.java
+++
b/kylin-spark-project/kylin-spark-classloader/src/main/java/org/apache/kylin/spark/classloader/SparkClassLoader.java
@@ -121,7 +121,6 @@ public class SparkClassLoader extends URLClassLoader {
for (File jar : jars) {
addURL(jar.toURI().toURL());
}
-
}
@Override
diff --git
a/kylin-spark-project/kylin-spark-common/src/main/java/org/apache/kylin/engine/spark/metadata/cube/PathManager.java
b/kylin-spark-project/kylin-spark-common/src/main/java/org/apache/kylin/engine/spark/metadata/cube/PathManager.java
index 0d34451..e30b2c0 100644
---
a/kylin-spark-project/kylin-spark-common/src/main/java/org/apache/kylin/engine/spark/metadata/cube/PathManager.java
+++
b/kylin-spark-project/kylin-spark-common/src/main/java/org/apache/kylin/engine/spark/metadata/cube/PathManager.java
@@ -59,7 +59,7 @@ public final class PathManager {
* Delete segment path
*/
public static boolean deleteSegmentParquetStoragePath(CubeInstance cube,
String segmentName, String identifier) throws IOException {
- if (cube == null || StringUtils.isNoneBlank(segmentName)||
StringUtils.isNoneBlank(identifier)) {
+ if (cube == null || StringUtils.isBlank(segmentName) ||
StringUtils.isBlank(identifier)) {
return false;
}
String path = getSegmentParquetStoragePath(cube, segmentName,
identifier);
diff --git
a/kylin-spark-project/kylin-spark-engine/src/main/java/org/apache/kylin/engine/spark/job/NSparkExecutable.java
b/kylin-spark-project/kylin-spark-engine/src/main/java/org/apache/kylin/engine/spark/job/NSparkExecutable.java
index fbfae2e..86e234a 100644
---
a/kylin-spark-project/kylin-spark-engine/src/main/java/org/apache/kylin/engine/spark/job/NSparkExecutable.java
+++
b/kylin-spark-project/kylin-spark-engine/src/main/java/org/apache/kylin/engine/spark/job/NSparkExecutable.java
@@ -400,9 +400,6 @@ public class NSparkExecutable extends AbstractExecutable {
String.format(Locale.ROOT, "%s:%s", APP_JAR_NAME,
Paths.get(kylinJobJar).getFileName().toString()) :
kylinJobJar);
- if (sparkConfs.containsKey("spark.sql.hive.metastore.jars")) {
- jars = jars + "," +
sparkConfs.get("spark.sql.hive.metastore.jars");
- }
String sparkUploadFiles =
config.sparkUploadFiles(isLocalMaster(sparkConfs), isYarnCluster);
if (StringUtils.isNotBlank(sparkUploadFiles)) {
sb.append("--files ").append(sparkUploadFiles).append(" ");
diff --git
a/kylin-spark-project/kylin-spark-engine/src/main/scala/org/apache/spark/conf/rule/SparkConfRule.scala
b/kylin-spark-project/kylin-spark-engine/src/main/scala/org/apache/spark/conf/rule/SparkConfRule.scala
index 403e872..781d1b0 100644
---
a/kylin-spark-project/kylin-spark-engine/src/main/scala/org/apache/spark/conf/rule/SparkConfRule.scala
+++
b/kylin-spark-project/kylin-spark-engine/src/main/scala/org/apache/spark/conf/rule/SparkConfRule.scala
@@ -107,8 +107,20 @@ class ExecutorInstancesRule extends SparkConfRule {
logInfo(s"The number of instances calculated by cuboids " +
s"is: $calculateExecutorInsByLayoutSize")
- val availableMem =
helper.getFetcher.fetchQueueAvailableResource(queue).available.memory
- val availableCore =
helper.getFetcher.fetchQueueAvailableResource(queue).available.vCores
+ var availableMem = Int.MaxValue
+ try {
+ availableMem =
helper.getFetcher.fetchQueueAvailableResource(queue).available.memory
+ } catch {
+ case throwable: Throwable =>
+ logWarning(s"Error when getting available memory from cluster, ignore
it.")
+ }
+ var availableCore = Int.MaxValue
+ try {
+ availableCore =
helper.getFetcher.fetchQueueAvailableResource(queue).available.vCores
+ } catch {
+ case throwable: Throwable =>
+ logWarning(s"Error when getting available cpus from cluster, ignore
it.")
+ }
val executorMem =
(Utils.byteStringAsMb(helper.getConf(SparkConfHelper.EXECUTOR_MEMORY)) +
Utils.byteStringAsMb(helper.getConf(SparkConfHelper.EXECUTOR_OVERHEAD)))
diff --git
a/kylin-spark-project/kylin-spark-engine/src/main/scala/org/apache/spark/utils/ResourceUtils.scala
b/kylin-spark-project/kylin-spark-engine/src/main/scala/org/apache/spark/utils/ResourceUtils.scala
index de6697b..edf4b21 100644
---
a/kylin-spark-project/kylin-spark-engine/src/main/scala/org/apache/spark/utils/ResourceUtils.scala
+++
b/kylin-spark-project/kylin-spark-engine/src/main/scala/org/apache/spark/utils/ResourceUtils.scala
@@ -67,8 +67,11 @@ object ResourceUtils extends Logging {
def checkResource(sparkConf: SparkConf, clusterInfo: ClusterInfoFetcher):
Boolean = {
val queue = sparkConf.get("spark.yarn.queue", "default")
- val driverMemory = (Utils.byteStringAsMb(sparkConf.get(DRIVER_MEMORY)) +
Utils.byteStringAsMb(sparkConf.get(DRIVER_OVERHEAD))).toInt
- val driverCores = sparkConf.get(DRIVER_CORES).toInt
+ var driverMemory = Utils.byteStringAsMb(sparkConf.get(DRIVER_MEMORY)).toInt
+ if (sparkConf.contains(DRIVER_OVERHEAD)) {
+ driverMemory = driverMemory +
Utils.byteStringAsMb(sparkConf.get(DRIVER_OVERHEAD)).toInt
+ }
+ val driverCores = sparkConf.get(DRIVER_CORES, "1").toInt
val queueAvailable =
minusDriverResource(clusterInfo.fetchQueueAvailableResource(queue),
driverMemory, driverCores)
val instances = sparkConf.get(EXECUTOR_INSTANCES).toInt
val executorMemory = (Utils.byteStringAsMb(sparkConf.get(EXECUTOR_MEMORY))
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 892cf9c..5d25d9c 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
@@ -135,6 +135,14 @@ object KylinSession extends Logging {
private lazy val conf: KylinConfig = KylinConfig.getInstanceFromEnv
+ def setDistJarFiles(sparkConf: SparkConf, key: String, value: String):
Unit = {
+ if (sparkConf.contains(key)) {
+ sparkConf.set(key, value + "," + sparkConf.get(key))
+ } else {
+ sparkConf.set(key, value)
+ }
+ }
+
def initSparkConf(sparkConf: SparkConf): SparkConf = {
if (sparkConf.getBoolean("user.kylin.session", false)) {
return sparkConf
@@ -172,12 +180,13 @@ object KylinSession extends Logging {
if (!"true".equalsIgnoreCase(System.getProperty("spark.local"))) {
if (sparkConf.get("spark.master").startsWith("yarn")) {
- sparkConf.set("spark.yarn.dist.jars",
+ setDistJarFiles(sparkConf, "spark.yarn.dist.jars",
KylinConfig.getInstanceFromEnv.getKylinParquetJobJarPath)
- sparkConf.set("spark.yarn.dist.files", conf.sparkUploadFiles())
+ setDistJarFiles(sparkConf, "spark.yarn.dist.files",
conf.sparkUploadFiles())
} else {
- sparkConf.set("spark.jars", conf.sparderJars)
- sparkConf.set("spark.files", conf.sparkUploadFiles())
+ setDistJarFiles(sparkConf, "spark.jars",
+ KylinConfig.getInstanceFromEnv.getKylinParquetJobJarPath)
+ setDistJarFiles(sparkConf, "spark.files", conf.sparkUploadFiles())
}
val fileName = KylinConfig.getInstanceFromEnv.getKylinParquetJobJarPath
diff --git a/pom.xml b/pom.xml
index d8f73be..1b794fd 100644
--- a/pom.xml
+++ b/pom.xml
@@ -2034,6 +2034,7 @@
<spark.version>3.1.1</spark.version>
<spark.version.dir>spark31</spark.version.dir>
<jackson.version>2.10.0</jackson.version>
+ <slf4j.version>1.7.30</slf4j.version>
</properties>
<build>
<plugins>
diff --git
a/server-base/src/main/java/org/apache/kylin/rest/controller/CubeController.java
b/server-base/src/main/java/org/apache/kylin/rest/controller/CubeController.java
index 23ad6a7..fff3faf 100644
---
a/server-base/src/main/java/org/apache/kylin/rest/controller/CubeController.java
+++
b/server-base/src/main/java/org/apache/kylin/rest/controller/CubeController.java
@@ -82,6 +82,7 @@ import org.apache.kylin.rest.service.ProjectService;
import org.apache.kylin.rest.service.QueryService;
import org.apache.kylin.rest.util.AclEvaluate;
import org.apache.kylin.rest.util.ValidateUtil;
+import org.apache.kylin.shaded.com.google.common.collect.Maps;
import org.slf4j.Logger;
import org.slf4j.LoggerFactory;
import org.springframework.beans.factory.annotation.Autowired;
@@ -174,7 +175,14 @@ public class CubeController extends BasicController {
@RequestMapping(value = "validEncodings", method = { RequestMethod.GET },
produces = { "application/json" })
@ResponseBody
public Map<String, Integer> getValidEncodings() {
- throw new UnsupportedOperationException("Unsupported in Kylin 4.0 .");
+ Map<String, Integer> encodings;
+ try {
+ encodings = DimensionEncodingFactory.getValidEncodings();
+ } catch (Exception e) {
+ logger.error("Error when getting valid encodings", e);
+ return Maps.newHashMap();
+ }
+ return encodings;
}
@RequestMapping(value = "/{cubeName}", method = { RequestMethod.GET },
produces = { "application/json" })