This is an automated email from the ASF dual-hosted git repository. zhangzc 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 45a7889 KYLIN-5059 Fix error when using different HDFS cluster in cube building (#1721) 45a7889 is described below commit 45a7889eccf8acf7bc14890a7b930a12adebd463 Author: zhengshengjun <74281684+zhengsheng...@users.noreply.github.com> AuthorDate: Wed Aug 11 09:53:15 2021 +0800 KYLIN-5059 Fix error when using different HDFS cluster in cube building (#1721) --- .../src/main/scala/org/apache/spark/dict/NBucketDictionary.java | 7 ++++--- .../apache/spark/sql/catalyst/expressions/KylinExpresssions.scala | 7 ++++--- 2 files changed, 8 insertions(+), 6 deletions(-) diff --git a/kylin-spark-project/kylin-spark-common/src/main/scala/org/apache/spark/dict/NBucketDictionary.java b/kylin-spark-project/kylin-spark-common/src/main/scala/org/apache/spark/dict/NBucketDictionary.java index 2e310ca..cd5caf4 100644 --- a/kylin-spark-project/kylin-spark-common/src/main/scala/org/apache/spark/dict/NBucketDictionary.java +++ b/kylin-spark-project/kylin-spark-common/src/main/scala/org/apache/spark/dict/NBucketDictionary.java @@ -57,10 +57,11 @@ public class NBucketDictionary { } this.relativeDictMap = new Object2LongOpenHashMap<>(); if (!StringUtils.isEmpty(skewDictStorageFile)) { - FileSystem fs = FileSystem.get(new Configuration()); - if (fs.exists(new Path(skewDictStorageFile))) { + Path skewedDictPath = new Path(skewDictStorageFile); + FileSystem fs = skewedDictPath.getFileSystem(new Configuration()); + if (fs.exists(skewedDictPath)) { Kryo kryo = new Kryo(); - Input input = new Input(fs.open(new Path(skewDictStorageFile))); + Input input = new Input(fs.open(skewedDictPath)); skewedDictMap = (Object2LongMap<String>) kryo.readClassAndObject(input); input.close(); } diff --git a/kylin-spark-project/kylin-spark-common/src/main/scala/org/apache/spark/sql/catalyst/expressions/KylinExpresssions.scala b/kylin-spark-project/kylin-spark-common/src/main/scala/org/apache/spark/sql/catalyst/expressions/KylinExpresssions.scala index 7a5b965..b47ff90 100644 --- a/kylin-spark-project/kylin-spark-common/src/main/scala/org/apache/spark/sql/catalyst/expressions/KylinExpresssions.scala +++ b/kylin-spark-project/kylin-spark-common/src/main/scala/org/apache/spark/sql/catalyst/expressions/KylinExpresssions.scala @@ -500,10 +500,11 @@ case class ScatterSkewData(left: Expression, right: Expression) extends BinaryEx | ${rand} = new java.util.Random(); | com.esotericsoftware.kryo.Kryo kryo = new com.esotericsoftware.kryo.Kryo(); | try { - | org.apache.hadoop.fs.FileSystem fs = org.apache.hadoop.fs.FileSystem.get(new org.apache.hadoop.conf.Configuration()); - | if (fs.exists(new org.apache.hadoop.fs.Path("${skewDataStorage}"))) { + | org.apache.hadoop.fs.Path skewDictPath = new org.apache.hadoop.fs.Path("${skewDataStorage}"); + | org.apache.hadoop.fs.FileSystem fs = skewDictPath.getFileSystem(new org.apache.hadoop.conf.Configuration()); + | if (fs.exists(skewDictPath)) { | com.esotericsoftware.kryo.io.Input input = new com.esotericsoftware.kryo.io.Input( - | fs.open(new org.apache.hadoop.fs.Path("${skewDataStorage}"))); + | fs.open(skewDictPath)); | ${skewData} = (it.unimi.dsi.fastutil.objects.Object2LongOpenHashMap<String>) kryo.readClassAndObject(input); | input.close(); | }