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

jiafengzheng pushed a commit to branch master
in repository 
https://gitbox.apache.org/repos/asf/incubator-doris-spark-connector.git


The following commit(s) were added to refs/heads/master by this push:
     new 4a66fff  [fix] Fix doris.read.field configuration does not take 
effect. (#20)
4a66fff is described below

commit 4a66fff145b6bf793a0d6accb1b24f49b981efc2
Author: smallhibiscus <[email protected]>
AuthorDate: Thu May 12 16:47:23 2022 +0800

    [fix] Fix doris.read.field configuration does not take effect. (#20)
    
    * Fix doris.read.field configuration does not take effect.
---
 .../org/apache/doris/spark/sql/SchemaUtils.scala   | 26 +++++++++++++++++-----
 .../apache/doris/spark/sql/TestSchemaUtils.scala   |  2 +-
 2 files changed, 22 insertions(+), 6 deletions(-)

diff --git 
a/spark-doris-connector/src/main/scala/org/apache/doris/spark/sql/SchemaUtils.scala
 
b/spark-doris-connector/src/main/scala/org/apache/doris/spark/sql/SchemaUtils.scala
index f45b9de..26d5c82 100644
--- 
a/spark-doris-connector/src/main/scala/org/apache/doris/spark/sql/SchemaUtils.scala
+++ 
b/spark-doris-connector/src/main/scala/org/apache/doris/spark/sql/SchemaUtils.scala
@@ -24,11 +24,13 @@ import org.apache.doris.spark.exception.DorisException
 import org.apache.doris.spark.rest.RestService
 import org.apache.doris.spark.rest.models.{Field, Schema}
 import org.apache.doris.thrift.TScanColumnDesc
-
+import org.apache.doris.spark.cfg.ConfigurationOptions.DORIS_READ_FIELD
 import org.apache.spark.sql.types._
 
 import org.slf4j.LoggerFactory
 
+import scala.collection.mutable
+
 private[spark] object SchemaUtils {
   private val logger = 
LoggerFactory.getLogger(SchemaUtils.getClass.getSimpleName.stripSuffix("$"))
 
@@ -39,7 +41,7 @@ private[spark] object SchemaUtils {
    */
   def discoverSchema(cfg: Settings): StructType = {
     val schema = discoverSchemaFromFe(cfg)
-    convertToStruct(schema)
+    convertToStruct(cfg.getProperty(DORIS_READ_FIELD), schema)
   }
 
   /**
@@ -56,10 +58,24 @@ private[spark] object SchemaUtils {
    * @param schema inner schema
    * @return Spark Catalyst StructType
    */
-  def convertToStruct(schema: Schema): StructType = {
+  def convertToStruct(dorisReadFields: String, schema: Schema): StructType = {
+    var fieldList = new Array[String](schema.size())
+    val fieldSet = new mutable.HashSet[String]()
     var fields = List[StructField]()
-    schema.getProperties.asScala.foreach(f =>
-      fields :+= DataTypes.createStructField(f.getName, 
getCatalystType(f.getType, f.getPrecision, f.getScale), true))
+    if (dorisReadFields != null && dorisReadFields.length > 0) {
+      fieldList = dorisReadFields.split(",")
+      for (field <- fieldList) {
+        fieldSet.add(field)
+      }
+      schema.getProperties.asScala.foreach(f =>
+        if (fieldSet.contains(f.getName)) {
+          fields :+= DataTypes.createStructField(f.getName, 
getCatalystType(f.getType, f.getPrecision, f.getScale), true)
+        })
+    } else {
+      schema.getProperties.asScala.foreach(f =>
+        fields :+= DataTypes.createStructField(f.getName, 
getCatalystType(f.getType, f.getPrecision, f.getScale), true)
+      )
+    }
     DataTypes.createStructType(fields.asJava)
   }
 
diff --git 
a/spark-doris-connector/src/test/scala/org/apache/doris/spark/sql/TestSchemaUtils.scala
 
b/spark-doris-connector/src/test/scala/org/apache/doris/spark/sql/TestSchemaUtils.scala
index 13004b4..0a1dd35 100644
--- 
a/spark-doris-connector/src/test/scala/org/apache/doris/spark/sql/TestSchemaUtils.scala
+++ 
b/spark-doris-connector/src/test/scala/org/apache/doris/spark/sql/TestSchemaUtils.scala
@@ -39,7 +39,7 @@ class TestSchemaUtils extends ExpectedExceptionTest {
     var fields = List[StructField]()
     fields :+= DataTypes.createStructField("k1", DataTypes.ByteType, true)
     fields :+= DataTypes.createStructField("k5", DataTypes.LongType, true)
-    val expected = DataTypes.createStructType(fields.asJava)
+    val expected = DataTypes.createStructType("k1,k5", fields.asJava)
     Assert.assertEquals(expected, SchemaUtils.convertToStruct(schema))
   }
 


---------------------------------------------------------------------
To unsubscribe, e-mail: [email protected]
For additional commands, e-mail: [email protected]

Reply via email to