You are viewing a plain text version of this content. The canonical link for it is here.
Posted to reviews@spark.apache.org by GitBox <gi...@apache.org> on 2022/12/22 00:52:19 UTC

[GitHub] [spark] techaddict opened a new pull request, #39164: [SPARK-41432][UI][SQL] Protobuf serializer for SparkPlanGraphWrapper

techaddict opened a new pull request, #39164:
URL: https://github.com/apache/spark/pull/39164

   ### What changes were proposed in this pull request?
   Add Protobuf serializer for SparkPlanGraphWrapper
   
   ### Why are the changes needed?
   Support fast and compact serialization/deserialization for SparkPlanGraphWrapper over RocksDB.
   
   ### Does this PR introduce any user-facing change?
   No
   
   ### How was this patch tested?
   New UT


-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


---------------------------------------------------------------------
To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org
For additional commands, e-mail: reviews-help@spark.apache.org


[GitHub] [spark] gengliangwang commented on a diff in pull request #39164: [SPARK-41432][UI][SQL] Protobuf serializer for SparkPlanGraphWrapper

Posted by GitBox <gi...@apache.org>.
gengliangwang commented on code in PR #39164:
URL: https://github.com/apache/spark/pull/39164#discussion_r1055270691


##########
sql/core/pom.xml:
##########
@@ -291,5 +297,68 @@
         </dependency>
       </dependencies>
     </profile>
+    <profile>

Review Comment:
   Thanks for looking into it!



-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


---------------------------------------------------------------------
To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org
For additional commands, e-mail: reviews-help@spark.apache.org


[GitHub] [spark] LuciferYang commented on a diff in pull request #39164: [SPARK-41432][UI][SQL] Protobuf serializer for SparkPlanGraphWrapper

Posted by GitBox <gi...@apache.org>.
LuciferYang commented on code in PR #39164:
URL: https://github.com/apache/spark/pull/39164#discussion_r1055412267


##########
sql/core/pom.xml:
##########
@@ -272,6 +278,23 @@
           </execution>
         </executions>
       </plugin>
+      <plugin>
+        <groupId>org.apache.maven.plugins</groupId>
+        <artifactId>maven-shade-plugin</artifactId>
+        <configuration>
+          <shadedArtifactAttached>false</shadedArtifactAttached>
+          <shadeTestJar>true</shadeTestJar>
+          <relocations>

Review Comment:
   should also copy relocation rule of guava to sql module



-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


---------------------------------------------------------------------
To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org
For additional commands, e-mail: reviews-help@spark.apache.org


[GitHub] [spark] gengliangwang commented on pull request #39164: [SPARK-41432][UI][SQL] Protobuf serializer for SparkPlanGraphWrapper

Posted by GitBox <gi...@apache.org>.
gengliangwang commented on PR #39164:
URL: https://github.com/apache/spark/pull/39164#issuecomment-1363417209

   @techaddict How about following https://github.com/apache/spark/pull/39139? In that way, we don't need to have another shading in the SQL module.


-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


---------------------------------------------------------------------
To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org
For additional commands, e-mail: reviews-help@spark.apache.org


[GitHub] [spark] gengliangwang commented on pull request #39164: [SPARK-41432][UI][SQL] Protobuf serializer for SparkPlanGraphWrapper

Posted by GitBox <gi...@apache.org>.
gengliangwang commented on PR #39164:
URL: https://github.com/apache/spark/pull/39164#issuecomment-1362605418

   @techaddict could you focus on this one first?
   


-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


---------------------------------------------------------------------
To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org
For additional commands, e-mail: reviews-help@spark.apache.org


[GitHub] [spark] techaddict commented on pull request #39164: [SPARK-41432][UI][SQL] Protobuf serializer for SparkPlanGraphWrapper

Posted by GitBox <gi...@apache.org>.
techaddict commented on PR #39164:
URL: https://github.com/apache/spark/pull/39164#issuecomment-1362662977

   @LuciferYang @gengliangwang sounds good
   


-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


---------------------------------------------------------------------
To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org
For additional commands, e-mail: reviews-help@spark.apache.org


[GitHub] [spark] techaddict commented on a diff in pull request #39164: [SPARK-41432][UI][SQL] Protobuf serializer for SparkPlanGraphWrapper

Posted by GitBox <gi...@apache.org>.
techaddict commented on code in PR #39164:
URL: https://github.com/apache/spark/pull/39164#discussion_r1055706951


##########
sql/core/src/main/protobuf/org/apache/spark/status/protobuf/store_types.proto:
##########
@@ -0,0 +1,56 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one or more
+ * contributor license agreements.  See the NOTICE file distributed with
+ * this work for additional information regarding copyright ownership.
+ * The ASF licenses this file to You under the Apache License, Version 2.0
+ * (the "License"); you may not use this file except in compliance with
+ * the License.  You may obtain a copy of the License at
+ *
+ *    http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+
+syntax = "proto3";
+package org.apache.spark.status.protobuf.sql;

Review Comment:
   Ya I don't think that will work.
   We can always reference the java files it creates so should get access to all the structures.



-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


---------------------------------------------------------------------
To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org
For additional commands, e-mail: reviews-help@spark.apache.org


[GitHub] [spark] LuciferYang commented on pull request #39164: [SPARK-41432][UI][SQL] Protobuf serializer for SparkPlanGraphWrapper

Posted by GitBox <gi...@apache.org>.
LuciferYang commented on PR #39164:
URL: https://github.com/apache/spark/pull/39164#issuecomment-1363734707

   Yes, GA only runs sbt tests now , while Maven only run build with Java 11 & 17, no tests.
   
   


-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


---------------------------------------------------------------------
To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org
For additional commands, e-mail: reviews-help@spark.apache.org


[GitHub] [spark] LuciferYang commented on a diff in pull request #39164: [SPARK-41432][UI][SQL] Protobuf serializer for SparkPlanGraphWrapper

Posted by GitBox <gi...@apache.org>.
LuciferYang commented on code in PR #39164:
URL: https://github.com/apache/spark/pull/39164#discussion_r1055025912


##########
sql/core/pom.xml:
##########
@@ -291,5 +297,68 @@
         </dependency>
       </dependencies>
     </profile>
+    <profile>

Review Comment:
   for SBT we can refer to 
   
   https://github.com/apache/spark/blob/73593d8670ca5dd933327039fc659dd6ef764963/project/SparkBuild.scala#L618-L631
   
   https://github.com/apache/spark/blob/73593d8670ca5dd933327039fc659dd6ef764963/project/SparkBuild.scala#L645-L654
   
   Otherwise, GA task may can't compile proto
   
   



##########
sql/core/pom.xml:
##########
@@ -291,5 +297,68 @@
         </dependency>
       </dependencies>
     </profile>
+    <profile>

Review Comment:
   for SBT we can refer to 
   
   https://github.com/apache/spark/blob/73593d8670ca5dd933327039fc659dd6ef764963/project/SparkBuild.scala#L618-L631
   
   https://github.com/apache/spark/blob/73593d8670ca5dd933327039fc659dd6ef764963/project/SparkBuild.scala#L645-L654
   
   Otherwise, GA task may can't compile proto file
   
   



-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


---------------------------------------------------------------------
To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org
For additional commands, e-mail: reviews-help@spark.apache.org


[GitHub] [spark] LuciferYang commented on a diff in pull request #39164: [SPARK-41432][UI][SQL] Protobuf serializer for SparkPlanGraphWrapper

Posted by GitBox <gi...@apache.org>.
LuciferYang commented on code in PR #39164:
URL: https://github.com/apache/spark/pull/39164#discussion_r1055021791


##########
sql/core/pom.xml:
##########
@@ -291,5 +297,68 @@
         </dependency>
       </dependencies>
     </profile>
+    <profile>

Review Comment:
   Would you mind adding to this pr? This is some necessary initial work for sql module @techaddict 
   
   both `pom.xml` and `SparkBuild.scala`
   
   
   
   



-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


---------------------------------------------------------------------
To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org
For additional commands, e-mail: reviews-help@spark.apache.org


[GitHub] [spark] LuciferYang commented on a diff in pull request #39164: [SPARK-41432][UI][SQL] Protobuf serializer for SparkPlanGraphWrapper

Posted by GitBox <gi...@apache.org>.
LuciferYang commented on code in PR #39164:
URL: https://github.com/apache/spark/pull/39164#discussion_r1055021791


##########
sql/core/pom.xml:
##########
@@ -291,5 +297,68 @@
         </dependency>
       </dependencies>
     </profile>
+    <profile>

Review Comment:
   Would you mind adding to this pr? This is some necessary initial work @techaddict 
   
   both `pom.xml` and `SparkBuild.scala`
   
   
   
   



-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


---------------------------------------------------------------------
To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org
For additional commands, e-mail: reviews-help@spark.apache.org


[GitHub] [spark] techaddict commented on pull request #39164: [SPARK-41432][UI][SQL] Protobuf serializer for SparkPlanGraphWrapper

Posted by GitBox <gi...@apache.org>.
techaddict commented on PR #39164:
URL: https://github.com/apache/spark/pull/39164#issuecomment-1363979055

   @LuciferYang I see the issue, too; it looks like the way around is to add a maven dependency


-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


---------------------------------------------------------------------
To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org
For additional commands, e-mail: reviews-help@spark.apache.org


[GitHub] [spark] LuciferYang commented on a diff in pull request #39164: [SPARK-41432][UI][SQL] Protobuf serializer for SparkPlanGraphWrapper

Posted by GitBox <gi...@apache.org>.
LuciferYang commented on code in PR #39164:
URL: https://github.com/apache/spark/pull/39164#discussion_r1055445244


##########
sql/core/src/main/protobuf/org/apache/spark/status/protobuf/store_types.proto:
##########
@@ -0,0 +1,56 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one or more
+ * contributor license agreements.  See the NOTICE file distributed with
+ * this work for additional information regarding copyright ownership.
+ * The ASF licenses this file to You under the Apache License, Version 2.0
+ * (the "License"); you may not use this file except in compliance with
+ * the License.  You may obtain a copy of the License at
+ *
+ *    http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+
+syntax = "proto3";
+package org.apache.spark.status.protobuf.sql;

Review Comment:
   seems impossible



-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


---------------------------------------------------------------------
To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org
For additional commands, e-mail: reviews-help@spark.apache.org


[GitHub] [spark] LuciferYang commented on a diff in pull request #39164: [SPARK-41432][UI][SQL] Protobuf serializer for SparkPlanGraphWrapper

Posted by GitBox <gi...@apache.org>.
LuciferYang commented on code in PR #39164:
URL: https://github.com/apache/spark/pull/39164#discussion_r1055018553


##########
sql/core/pom.xml:
##########
@@ -291,5 +297,68 @@
         </dependency>
       </dependencies>
     </profile>
+    <profile>

Review Comment:
   I think we should also shaded and relocation `protobuf-java` in sql module, like 
   
   https://github.com/apache/spark/blob/73593d8670ca5dd933327039fc659dd6ef764963/core/pom.xml#L690-L696



-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


---------------------------------------------------------------------
To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org
For additional commands, e-mail: reviews-help@spark.apache.org


[GitHub] [spark] LuciferYang commented on a diff in pull request #39164: [SPARK-41432][UI][SQL] Protobuf serializer for SparkPlanGraphWrapper

Posted by GitBox <gi...@apache.org>.
LuciferYang commented on code in PR #39164:
URL: https://github.com/apache/spark/pull/39164#discussion_r1057609250


##########
sql/core/src/main/scala/org/apache/spark/status/protobuf/sql/SparkPlanGraphWrapperSerializer.scala:
##########
@@ -0,0 +1,152 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one or more
+ * contributor license agreements.  See the NOTICE file distributed with
+ * this work for additional information regarding copyright ownership.
+ * The ASF licenses this file to You under the Apache License, Version 2.0
+ * (the "License"); you may not use this file except in compliance with
+ * the License.  You may obtain a copy of the License at
+ *
+ *    http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+
+package org.apache.spark.status.protobuf.sql
+
+import collection.JavaConverters._
+
+import org.apache.spark.sql.execution.ui.{SparkPlanGraphClusterWrapper, SparkPlanGraphEdge, SparkPlanGraphNode, SparkPlanGraphNodeWrapper, SparkPlanGraphWrapper, SQLPlanMetric}
+import org.apache.spark.status.protobuf.ProtobufSerDe
+import org.apache.spark.status.protobuf.StoreTypes
+
+class SparkPlanGraphWrapperSerializer extends ProtobufSerDe {
+
+  override val supportClass: Class[_] = classOf[SparkPlanGraphWrapper]
+
+  override def serialize(input: Any): Array[Byte] =
+    serialize(input.asInstanceOf[SparkPlanGraphWrapper])
+
+  private def serialize(plan: SparkPlanGraphWrapper): Array[Byte] = {
+    val builder = StoreTypes.SparkPlanGraphWrapper.newBuilder()
+    builder.setExecutionId(plan.executionId)
+    plan.nodes.foreach { node =>
+      builder.addNodes(serializeSparkPlanGraphNodeWrapper(node))
+    }
+    plan.edges.foreach {edge =>
+      builder.addEdges(serializeSparkPlanGraphEdge(edge))
+    }
+    builder.build().toByteArray
+  }
+
+  def deserialize(bytes: Array[Byte]): SparkPlanGraphWrapper = {
+    val wrapper = StoreTypes.SparkPlanGraphWrapper.parseFrom(bytes)
+    new SparkPlanGraphWrapper(
+      executionId = wrapper.getExecutionId,
+      nodes = wrapper.getNodesList.asScala.map(deserializeSparkPlanGraphNodeWrapper).toSeq,
+      edges = wrapper.getEdgesList.asScala.map(deserializeSparkPlanGraphEdge).toSeq
+    )
+  }
+
+  private def serializeSparkPlanGraphNodeWrapper(input: SparkPlanGraphNodeWrapper):
+    StoreTypes.SparkPlanGraphNodeWrapper = {
+
+    val builder = StoreTypes.SparkPlanGraphNodeWrapper.newBuilder()
+    builder.setNode(serializeSparkPlanGraphNode(input.node))
+    builder.setCluster(serializeSparkPlanGraphClusterWrapper(input.cluster))
+    builder.build()
+  }
+
+  private def deserializeSparkPlanGraphNodeWrapper(input: StoreTypes.SparkPlanGraphNodeWrapper):
+    SparkPlanGraphNodeWrapper = {
+
+    new SparkPlanGraphNodeWrapper(
+      node = deserializeSparkPlanGraphNode(input.getNode),
+      cluster = deserializeSparkPlanGraphClusterWrapper(input.getCluster)
+    )
+  }
+
+  private def serializeSparkPlanGraphEdge(edge: SparkPlanGraphEdge):
+    StoreTypes.SparkPlanGraphEdge = {
+    val builder = StoreTypes.SparkPlanGraphEdge.newBuilder()
+    builder.setFromId(edge.fromId)
+    builder.setToId(edge.toId)
+    builder.build()
+  }
+
+  private def deserializeSparkPlanGraphEdge(edge: StoreTypes.SparkPlanGraphEdge):
+    SparkPlanGraphEdge = {
+    SparkPlanGraphEdge(
+      fromId = edge.getFromId,
+      toId = edge.getToId)
+  }
+
+  private def serializeSparkPlanGraphNode(node: SparkPlanGraphNode):
+    StoreTypes.SparkPlanGraphNode = {
+    val builder = StoreTypes.SparkPlanGraphNode.newBuilder()
+    builder.setId(node.id)
+    builder.setName(node.name)
+    builder.setDesc(node.desc)
+    node.metrics.foreach { metric =>
+      builder.addMetrics(serializeSQLPlanMetric(metric))
+    }
+    builder.build()
+  }
+
+  private def deserializeSparkPlanGraphNode(node: StoreTypes.SparkPlanGraphNode):
+    SparkPlanGraphNode = {
+
+    new SparkPlanGraphNode(
+      id = node.getId,
+      name = node.getName,
+      desc = node.getDesc,
+      metrics = node.getMetricsList.asScala.map(deserializeSQLPlanMetric).toSeq
+    )
+  }
+
+  private def serializeSparkPlanGraphClusterWrapper(cluster: SparkPlanGraphClusterWrapper):
+    StoreTypes.SparkPlanGraphClusterWrapper = {
+    val builder = StoreTypes.SparkPlanGraphClusterWrapper.newBuilder()
+    builder.setId(cluster.id)
+    builder.setName(cluster.name)
+    builder.setDesc(cluster.desc)
+    cluster.nodes.foreach { node =>
+      builder.addNodes(serializeSparkPlanGraphNodeWrapper(node))
+    }
+    cluster.metrics.foreach { metric =>
+      builder.addMetrics(serializeSQLPlanMetric(metric))
+    }
+    builder.build()
+  }
+
+  private def deserializeSparkPlanGraphClusterWrapper(
+    cluster: StoreTypes.SparkPlanGraphClusterWrapper): SparkPlanGraphClusterWrapper = {
+
+    new SparkPlanGraphClusterWrapper(
+      id = cluster.getId,
+      name = cluster.getName,
+      desc = cluster.getDesc,
+      nodes = cluster.getNodesList.asScala.map(deserializeSparkPlanGraphNodeWrapper).toSeq,
+      metrics = cluster.getMetricsList.asScala.map(deserializeSQLPlanMetric).toSeq
+    )
+  }
+
+  private def serializeSQLPlanMetric(metric: SQLPlanMetric): StoreTypes.SQLPlanMetric = {

Review Comment:
   could you reuse existing `SQLPlanMetricSerializer`



-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


---------------------------------------------------------------------
To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org
For additional commands, e-mail: reviews-help@spark.apache.org


[GitHub] [spark] LuciferYang commented on a diff in pull request #39164: [SPARK-41432][UI][SQL] Protobuf serializer for SparkPlanGraphWrapper

Posted by GitBox <gi...@apache.org>.
LuciferYang commented on code in PR #39164:
URL: https://github.com/apache/spark/pull/39164#discussion_r1055263888


##########
sql/core/pom.xml:
##########
@@ -291,5 +297,68 @@
         </dependency>
       </dependencies>
     </profile>
+    <profile>

Review Comment:
   https://github.com/apache/spark/blob/e56f31d5e764cd68afed87675e63be7f4fff3e54/pom.xml#L821-L831
   
   From the comments, maybe we can try to remove the old protobuf dependency. mesos uses the  `shaded-protobuf` one. Let me do some tests with different hadoop profiles first
   
   
   
    



-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


---------------------------------------------------------------------
To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org
For additional commands, e-mail: reviews-help@spark.apache.org


[GitHub] [spark] LuciferYang commented on pull request #39164: [SPARK-41432][UI][SQL] Protobuf serializer for SparkPlanGraphWrapper

Posted by GitBox <gi...@apache.org>.
LuciferYang commented on PR #39164:
URL: https://github.com/apache/spark/pull/39164#issuecomment-1362574305

   Since this pr will include some initialization work of the sql module, I think we should promote this first to facilitate the submission of other pr related to the sql module.


-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


---------------------------------------------------------------------
To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org
For additional commands, e-mail: reviews-help@spark.apache.org


[GitHub] [spark] codecov-commenter commented on pull request #39164: [SPARK-41432][UI][SQL] Protobuf serializer for SparkPlanGraphWrapper

Posted by GitBox <gi...@apache.org>.
codecov-commenter commented on PR #39164:
URL: https://github.com/apache/spark/pull/39164#issuecomment-1362724410

   # [Codecov](https://codecov.io/gh/apache/spark/pull/39164?src=pr&el=h1&utm_medium=referral&utm_source=github&utm_content=comment&utm_campaign=pr+comments&utm_term=The+Apache+Software+Foundation) Report
   > Merging [#39164](https://codecov.io/gh/apache/spark/pull/39164?src=pr&el=desc&utm_medium=referral&utm_source=github&utm_content=comment&utm_campaign=pr+comments&utm_term=The+Apache+Software+Foundation) (e56f31d) into [master](https://codecov.io/gh/apache/spark/commit/d94642625bc701f2bf76a0e1d97aa29212d961bd?el=desc&utm_medium=referral&utm_source=github&utm_content=comment&utm_campaign=pr+comments&utm_term=The+Apache+Software+Foundation) (d946426) will **decrease** coverage by `21.92%`.
   > The diff coverage is `80.25%`.
   
   > :exclamation: Current head e56f31d differs from pull request most recent head 743cd10. Consider uploading reports for the commit 743cd10 to get more accurate results
   
   ```diff
   @@             Coverage Diff             @@
   ##           master   #39164       +/-   ##
   ===========================================
   - Coverage   90.88%   68.96%   -21.93%     
   ===========================================
     Files         340      191      -149     
     Lines       75912    52313    -23599     
     Branches    10349     7496     -2853     
   ===========================================
   - Hits        68996    36076    -32920     
   - Misses       5330    14998     +9668     
   + Partials     1586     1239      -347     
   ```
   
   | Flag | Coverage Δ | |
   |---|---|---|
   | unittests | `68.94% <80.25%> (-21.93%)` | :arrow_down: |
   
   Flags with carried forward coverage won't be shown. [Click here](https://docs.codecov.io/docs/carryforward-flags?utm_medium=referral&utm_source=github&utm_content=comment&utm_campaign=pr+comments&utm_term=The+Apache+Software+Foundation#carryforward-flags-in-the-pull-request-comment) to find out more.
   
   | [Impacted Files](https://codecov.io/gh/apache/spark/pull/39164?src=pr&el=tree&utm_medium=referral&utm_source=github&utm_content=comment&utm_campaign=pr+comments&utm_term=The+Apache+Software+Foundation) | Coverage Δ | |
   |---|---|---|
   | [python/pyspark/sql/readwriter.py](https://codecov.io/gh/apache/spark/pull/39164/diff?src=pr&el=tree&utm_medium=referral&utm_source=github&utm_content=comment&utm_campaign=pr+comments&utm_term=The+Apache+Software+Foundation#diff-cHl0aG9uL3B5c3Bhcmsvc3FsL3JlYWR3cml0ZXIucHk=) | `47.64% <ø> (-28.24%)` | :arrow_down: |
   | [python/pyspark/testing/connectutils.py](https://codecov.io/gh/apache/spark/pull/39164/diff?src=pr&el=tree&utm_medium=referral&utm_source=github&utm_content=comment&utm_campaign=pr+comments&utm_term=The+Apache+Software+Foundation#diff-cHl0aG9uL3B5c3BhcmsvdGVzdGluZy9jb25uZWN0dXRpbHMucHk=) | `90.27% <ø> (-0.14%)` | :arrow_down: |
   | [...ython/pyspark/sql/connect/proto/expressions\_pb2.py](https://codecov.io/gh/apache/spark/pull/39164/diff?src=pr&el=tree&utm_medium=referral&utm_source=github&utm_content=comment&utm_campaign=pr+comments&utm_term=The+Apache+Software+Foundation#diff-cHl0aG9uL3B5c3Bhcmsvc3FsL2Nvbm5lY3QvcHJvdG8vZXhwcmVzc2lvbnNfcGIyLnB5) | `52.32% <15.90%> (-1.20%)` | :arrow_down: |
   | [python/pyspark/sql/session.py](https://codecov.io/gh/apache/spark/pull/39164/diff?src=pr&el=tree&utm_medium=referral&utm_source=github&utm_content=comment&utm_campaign=pr+comments&utm_term=The+Apache+Software+Foundation#diff-cHl0aG9uL3B5c3Bhcmsvc3FsL3Nlc3Npb24ucHk=) | `66.24% <50.00%> (-15.99%)` | :arrow_down: |
   | [python/pyspark/sql/connect/window.py](https://codecov.io/gh/apache/spark/pull/39164/diff?src=pr&el=tree&utm_medium=referral&utm_source=github&utm_content=comment&utm_campaign=pr+comments&utm_term=The+Apache+Software+Foundation#diff-cHl0aG9uL3B5c3Bhcmsvc3FsL2Nvbm5lY3Qvd2luZG93LnB5) | `65.48% <65.48%> (ø)` | |
   | [python/pyspark/sql/connect/session.py](https://codecov.io/gh/apache/spark/pull/39164/diff?src=pr&el=tree&utm_medium=referral&utm_source=github&utm_content=comment&utm_campaign=pr+comments&utm_term=The+Apache+Software+Foundation#diff-cHl0aG9uL3B5c3Bhcmsvc3FsL2Nvbm5lY3Qvc2Vzc2lvbi5weQ==) | `85.58% <85.71%> (+0.40%)` | :arrow_up: |
   | [python/pyspark/sql/connect/functions.py](https://codecov.io/gh/apache/spark/pull/39164/diff?src=pr&el=tree&utm_medium=referral&utm_source=github&utm_content=comment&utm_campaign=pr+comments&utm_term=The+Apache+Software+Foundation#diff-cHl0aG9uL3B5c3Bhcmsvc3FsL2Nvbm5lY3QvZnVuY3Rpb25zLnB5) | `87.55% <91.83%> (+0.82%)` | :arrow_up: |
   | [python/pyspark/sql/connect/column.py](https://codecov.io/gh/apache/spark/pull/39164/diff?src=pr&el=tree&utm_medium=referral&utm_source=github&utm_content=comment&utm_campaign=pr+comments&utm_term=The+Apache+Software+Foundation#diff-cHl0aG9uL3B5c3Bhcmsvc3FsL2Nvbm5lY3QvY29sdW1uLnB5) | `88.45% <92.75%> (+0.24%)` | :arrow_up: |
   | [python/pyspark/sql/connect/readwriter.py](https://codecov.io/gh/apache/spark/pull/39164/diff?src=pr&el=tree&utm_medium=referral&utm_source=github&utm_content=comment&utm_campaign=pr+comments&utm_term=The+Apache+Software+Foundation#diff-cHl0aG9uL3B5c3Bhcmsvc3FsL2Nvbm5lY3QvcmVhZHdyaXRlci5weQ==) | `67.33% <100.00%> (+4.69%)` | :arrow_up: |
   | [...n/pyspark/sql/tests/connect/test\_connect\_column.py](https://codecov.io/gh/apache/spark/pull/39164/diff?src=pr&el=tree&utm_medium=referral&utm_source=github&utm_content=comment&utm_campaign=pr+comments&utm_term=The+Apache+Software+Foundation#diff-cHl0aG9uL3B5c3Bhcmsvc3FsL3Rlc3RzL2Nvbm5lY3QvdGVzdF9jb25uZWN0X2NvbHVtbi5weQ==) | `96.52% <100.00%> (+0.48%)` | :arrow_up: |
   | ... and [282 more](https://codecov.io/gh/apache/spark/pull/39164/diff?src=pr&el=tree-more&utm_medium=referral&utm_source=github&utm_content=comment&utm_campaign=pr+comments&utm_term=The+Apache+Software+Foundation) | |
   
   :mega: We’re building smart automated test selection to slash your CI/CD build times. [Learn more](https://about.codecov.io/iterative-testing/?utm_medium=referral&utm_source=github&utm_content=comment&utm_campaign=pr+comments&utm_term=The+Apache+Software+Foundation)
   


-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


---------------------------------------------------------------------
To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org
For additional commands, e-mail: reviews-help@spark.apache.org


[GitHub] [spark] techaddict commented on pull request #39164: [SPARK-41432][UI][SQL] Protobuf serializer for SparkPlanGraphWrapper

Posted by GitBox <gi...@apache.org>.
techaddict commented on PR #39164:
URL: https://github.com/apache/spark/pull/39164#issuecomment-1363445977

   @gengliangwang done


-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


---------------------------------------------------------------------
To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org
For additional commands, e-mail: reviews-help@spark.apache.org


[GitHub] [spark] LuciferYang commented on a diff in pull request #39164: [SPARK-41432][UI][SQL] Protobuf serializer for SparkPlanGraphWrapper

Posted by GitBox <gi...@apache.org>.
LuciferYang commented on code in PR #39164:
URL: https://github.com/apache/spark/pull/39164#discussion_r1058147319


##########
sql/core/src/main/scala/org/apache/spark/status/protobuf/sql/SparkPlanGraphWrapperSerializer.scala:
##########
@@ -0,0 +1,136 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one or more
+ * contributor license agreements.  See the NOTICE file distributed with
+ * this work for additional information regarding copyright ownership.
+ * The ASF licenses this file to You under the Apache License, Version 2.0
+ * (the "License"); you may not use this file except in compliance with
+ * the License.  You may obtain a copy of the License at
+ *
+ *    http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+
+package org.apache.spark.status.protobuf.sql
+
+import collection.JavaConverters._
+
+import org.apache.spark.sql.execution.ui.{SparkPlanGraphClusterWrapper, SparkPlanGraphEdge, SparkPlanGraphNode, SparkPlanGraphNodeWrapper, SparkPlanGraphWrapper}
+import org.apache.spark.status.protobuf.ProtobufSerDe
+import org.apache.spark.status.protobuf.StoreTypes
+
+class SparkPlanGraphWrapperSerializer extends ProtobufSerDe {
+
+  override val supportClass: Class[_] = classOf[SparkPlanGraphWrapper]
+
+  override def serialize(input: Any): Array[Byte] =

Review Comment:
   same suggestion as https://github.com/apache/spark/pull/39110#discussion_r1057939206



-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


---------------------------------------------------------------------
To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org
For additional commands, e-mail: reviews-help@spark.apache.org


[GitHub] [spark] gengliangwang closed pull request #39164: [SPARK-41432][UI][SQL] Protobuf serializer for SparkPlanGraphWrapper

Posted by GitBox <gi...@apache.org>.
gengliangwang closed pull request #39164: [SPARK-41432][UI][SQL] Protobuf serializer for SparkPlanGraphWrapper
URL: https://github.com/apache/spark/pull/39164


-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


---------------------------------------------------------------------
To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org
For additional commands, e-mail: reviews-help@spark.apache.org


[GitHub] [spark] techaddict commented on pull request #39164: [SPARK-41432][UI][SQL] Protobuf serializer for SparkPlanGraphWrapper

Posted by GitBox <gi...@apache.org>.
techaddict commented on PR #39164:
URL: https://github.com/apache/spark/pull/39164#issuecomment-1364435802

   @gengliangwang @LuciferYang  rebased this, and its ready for review


-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


---------------------------------------------------------------------
To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org
For additional commands, e-mail: reviews-help@spark.apache.org


[GitHub] [spark] gengliangwang commented on pull request #39164: [SPARK-41432][UI][SQL] Protobuf serializer for SparkPlanGraphWrapper

Posted by GitBox <gi...@apache.org>.
gengliangwang commented on PR #39164:
URL: https://github.com/apache/spark/pull/39164#issuecomment-1363732588

   @LuciferYang This PR passes without the protobuf-java dep. Probably there is no maven build in GA tests?


-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


---------------------------------------------------------------------
To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org
For additional commands, e-mail: reviews-help@spark.apache.org


[GitHub] [spark] gengliangwang commented on a diff in pull request #39164: [SPARK-41432][UI][SQL] Protobuf serializer for SparkPlanGraphWrapper

Posted by GitBox <gi...@apache.org>.
gengliangwang commented on code in PR #39164:
URL: https://github.com/apache/spark/pull/39164#discussion_r1055256190


##########
sql/core/pom.xml:
##########
@@ -291,5 +297,68 @@
         </dependency>
       </dependencies>
     </profile>
+    <profile>

Review Comment:
   The old protobuf deps in root pom causes so many troubles...



-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


---------------------------------------------------------------------
To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org
For additional commands, e-mail: reviews-help@spark.apache.org


[GitHub] [spark] LuciferYang commented on a diff in pull request #39164: [SPARK-41432][UI][SQL] Protobuf serializer for SparkPlanGraphWrapper

Posted by GitBox <gi...@apache.org>.
LuciferYang commented on code in PR #39164:
URL: https://github.com/apache/spark/pull/39164#discussion_r1055422644


##########
sql/core/src/main/protobuf/org/apache/spark/status/protobuf/store_types.proto:
##########
@@ -0,0 +1,56 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one or more
+ * contributor license agreements.  See the NOTICE file distributed with
+ * this work for additional information regarding copyright ownership.
+ * The ASF licenses this file to You under the Apache License, Version 2.0
+ * (the "License"); you may not use this file except in compliance with
+ * the License.  You may obtain a copy of the License at
+ *
+ *    http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+
+syntax = "proto3";
+package org.apache.spark.status.protobuf.sql;

Review Comment:
   a newbie question: Can we import the `store_types.proto` file defined in the core module here? Otherwise, we need to repeatedly define some data structures in the future , such as `enum JobExecutionStatus`.



-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


---------------------------------------------------------------------
To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org
For additional commands, e-mail: reviews-help@spark.apache.org


[GitHub] [spark] gengliangwang commented on pull request #39164: [SPARK-41432][UI][SQL] Protobuf serializer for SparkPlanGraphWrapper

Posted by GitBox <gi...@apache.org>.
gengliangwang commented on PR #39164:
URL: https://github.com/apache/spark/pull/39164#issuecomment-1366979568

   Thanks, merging to master


-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


---------------------------------------------------------------------
To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org
For additional commands, e-mail: reviews-help@spark.apache.org


[GitHub] [spark] LuciferYang commented on a diff in pull request #39164: [SPARK-41432][UI][SQL] Protobuf serializer for SparkPlanGraphWrapper

Posted by GitBox <gi...@apache.org>.
LuciferYang commented on code in PR #39164:
URL: https://github.com/apache/spark/pull/39164#discussion_r1055018553


##########
sql/core/pom.xml:
##########
@@ -291,5 +297,68 @@
         </dependency>
       </dependencies>
     </profile>
+    <profile>

Review Comment:
   I think we should also shaded and relocation `protobuf-java` in sql module, like 
   
   https://github.com/LuciferYang/spark/blob/ef9c8e0d045576fb325ef337319fe6d59b7ce858/core/pom.xml#L691-L694



-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


---------------------------------------------------------------------
To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org
For additional commands, e-mail: reviews-help@spark.apache.org


[GitHub] [spark] techaddict commented on pull request #39164: [SPARK-41432][UI][SQL] Protobuf serializer for SparkPlanGraphWrapper

Posted by GitBox <gi...@apache.org>.
techaddict commented on PR #39164:
URL: https://github.com/apache/spark/pull/39164#issuecomment-1363341279

   @gengliangwang @LuciferYang addressed all the comments


-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


---------------------------------------------------------------------
To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org
For additional commands, e-mail: reviews-help@spark.apache.org


[GitHub] [spark] LuciferYang commented on pull request #39164: [SPARK-41432][UI][SQL] Protobuf serializer for SparkPlanGraphWrapper

Posted by GitBox <gi...@apache.org>.
LuciferYang commented on PR #39164:
URL: https://github.com/apache/spark/pull/39164#issuecomment-1363775929

   Run 
   
   ```
   gh pr checkout 39164  
   mvn clean test -pl sql/core -am -Dtest=none -DwildcardSuites=org.apache.spark.status.protobuf.sql.KVStoreProtobufSerializerSuite
   ```
   
   locally
   
   ```
   [INFO] --- scala-maven-plugin:4.8.0:compile (scala-compile-first) @ spark-sql_2.12 ---
   [INFO] Compiler bridge file: /Users/yangjie01/.sbt/1.0/zinc/org.scala-sbt/org.scala-sbt-compiler-bridge_2.12-1.8.0-bin_2.12.17__52.0-1.8.0_20221110T195421.jar
   [INFO] compiler plugin: BasicArtifact(com.github.ghik,silencer-plugin_2.12.17,1.7.10,null)
   [INFO] compiling 576 Scala sources and 75 Java sources to /Users/yangjie01/SourceCode/git/spark-mine-12/sql/core/target/scala-2.12/classes ...
   [ERROR] [Error] /Users/yangjie01/SourceCode/git/spark-mine-12/sql/core/src/main/scala/org/apache/spark/status/protobuf/sql/SparkPlanGraphWrapperSerializer.scala:34: Class com.google.protobuf.GeneratedMessageV3 not found - continuing with a stub.
   [ERROR] [Error] : Unable to locate class corresponding to inner class entry for Builder in owner com.google.protobuf.GeneratedMessageV3
   [ERROR] [Error] /Users/yangjie01/SourceCode/git/spark-mine-12/sql/core/src/main/scala/org/apache/spark/status/protobuf/sql/SparkPlanGraphWrapperSerializer.scala:37: Class com.google.protobuf.GeneratedMessageV3 not found - continuing with a stub.
   [ERROR] [Error] /Users/yangjie01/SourceCode/git/spark-mine-12/sql/core/src/main/scala/org/apache/spark/status/protobuf/sql/SparkPlanGraphWrapperSerializer.scala:40: Class com.google.protobuf.GeneratedMessageV3 not found - continuing with a stub.
   [ERROR] [Error] /Users/yangjie01/SourceCode/git/spark-mine-12/sql/core/src/main/scala/org/apache/spark/status/protobuf/sql/SparkPlanGraphWrapperSerializer.scala:42: value toByteArray is not a member of org.apache.spark.status.protobuf.StoreTypes.SparkPlanGraphWrapper
   [ERROR] [Error] : Unable to locate class corresponding to inner class entry for Builder in owner com.google.protobuf.GeneratedMessageV3
   [ERROR] [Error] /Users/yangjie01/SourceCode/git/spark-mine-12/sql/core/src/main/scala/org/apache/spark/status/protobuf/sql/SparkPlanGraphWrapperSerializer.scala:58: Class com.google.protobuf.GeneratedMessageV3 not found - continuing with a stub.
   [ERROR] [Error] /Users/yangjie01/SourceCode/git/spark-mine-12/sql/core/src/main/scala/org/apache/spark/status/protobuf/sql/SparkPlanGraphWrapperSerializer.scala:59: Class com.google.protobuf.GeneratedMessageV3 not found - continuing with a stub.
   [ERROR] [Error] : Unable to locate class corresponding to inner class entry for Builder in owner com.google.protobuf.GeneratedMessageV3
   [ERROR] [Error] : Unable to locate class corresponding to inner class entry for Builder in owner com.google.protobuf.GeneratedMessageV3
   [ERROR] [Error] /Users/yangjie01/SourceCode/git/spark-mine-12/sql/core/src/main/scala/org/apache/spark/status/protobuf/sql/SparkPlanGraphWrapperSerializer.scala:94: Class com.google.protobuf.GeneratedMessageV3 not found - continuing with a stub.
   [ERROR] [Error] : Unable to locate class corresponding to inner class entry for Builder in owner com.google.protobuf.GeneratedMessageV3
   [ERROR] [Error] : Unable to locate class corresponding to inner class entry for Builder in owner com.google.protobuf.GeneratedMessageV3
   [ERROR] 13 errors found
   ```


-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


---------------------------------------------------------------------
To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org
For additional commands, e-mail: reviews-help@spark.apache.org


[GitHub] [spark] LuciferYang commented on a diff in pull request #39164: [SPARK-41432][UI][SQL] Protobuf serializer for SparkPlanGraphWrapper

Posted by GitBox <gi...@apache.org>.
LuciferYang commented on code in PR #39164:
URL: https://github.com/apache/spark/pull/39164#discussion_r1055263888


##########
sql/core/pom.xml:
##########
@@ -291,5 +297,68 @@
         </dependency>
       </dependencies>
     </profile>
+    <profile>

Review Comment:
   https://github.com/apache/spark/blob/e56f31d5e764cd68afed87675e63be7f4fff3e54/pom.xml#L821-L831
   
   From the comments, maybe we can try to remove the old protobuf dependency. mesos uses the  `shaded-protobuf` one now. Let me do some tests with different hadoop profiles first
   
   
   
    



-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


---------------------------------------------------------------------
To unsubscribe, e-mail: reviews-unsubscribe@spark.apache.org
For additional commands, e-mail: reviews-help@spark.apache.org