You are viewing a plain text version of this content. The canonical link for it is here.
Posted to reviews@spark.apache.org by davies <gi...@git.apache.org> on 2016/05/19 22:07:16 UTC

[GitHub] spark pull request: explain of whole stage codegen

GitHub user davies opened a pull request:

    https://github.com/apache/spark/pull/13204

    explain of whole stage codegen

    ## What changes were proposed in this pull request?
    
    Currently, the explain of a query with whole-stage codegen looks like this
    ```
    >>> df = sqlCtx.range(1000);df2 = sqlCtx.range(1000);df.join(pyspark.sql.functions.broadcast(df2), 'id').explain()
    == Physical Plan ==
    WholeStageCodegen
    :  +- Project [id#1L]
    :     +- BroadcastHashJoin [id#1L], [id#4L], Inner, BuildRight, None
    :        :- Range 0, 1, 4, 1000, [id#1L]
    :        +- INPUT
    +- BroadcastExchange HashedRelationBroadcastMode(List(input[0, bigint]))
       +- WholeStageCodegen
          :  +- Range 0, 1, 4, 1000, [id#4L]
    ```
    
    The problem is that the plan looks much different than logical plan, make us hard to understand the plan (especially when the logical plan is not showed together).
    
    This PR will change it to:
    
    ```
    >>> df = sqlCtx.range(1000);df2 = sqlCtx.range(1000);df.join(pyspark.sql.functions.broadcast(df2), 'id').explain()
    == Physical Plan ==
    WholeStageCodegen
    +- Project [id#0L]
       +- BroadcastHashJoin [id#0L], [id#3L], Inner, BuildRight, None
          :- Range 0, 1, 4, 1000, [id#0L]
          +- CodegenInput
             +- BroadcastExchange HashedRelationBroadcastMode(List(input[0, bigint, false]))
                +- WholeStageCodegen
                   +- Range 0, 1, 4, 1000, [id#3L]
    ```
    
    This one is easier to understand the plan, but a little bit harder to find out which parts will be codegen together, we need to find all the operators between WholeStageCodegen and CodegenInput. Hopefully we do need to pay much attention on this, it's probably OK.
    
    ## How was this patch tested?
    
    Manually ran some queries and check the explain.
    
    


You can merge this pull request into a Git repository by running:

    $ git pull https://github.com/davies/spark explain_codegen

Alternatively you can review and apply these changes as the patch at:

    https://github.com/apache/spark/pull/13204.patch

To close this pull request, make a commit to your master/trunk branch
with (at least) the following in the commit message:

    This closes #13204
    
----
commit db5ad6211da458bed0ac66f9c93d87ae5b87ef16
Author: Davies Liu <da...@databricks.com>
Date:   2016-05-19T21:57:29Z

    explain of whole stage codegen

----


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

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


[GitHub] spark pull request: explain of whole stage codegen

Posted by yucai <gi...@git.apache.org>.
Github user yucai commented on the pull request:

    https://github.com/apache/spark/pull/13204#issuecomment-220505548
  
    Hi Davis,
    
    In your comments "Hopefully we do need to pay much attention on this", do you mean "we do not need to pay much attention..."?


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

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


[GitHub] spark pull request: explain of whole stage codegen

Posted by SparkQA <gi...@git.apache.org>.
Github user SparkQA commented on the pull request:

    https://github.com/apache/spark/pull/13204#issuecomment-220466925
  
    **[Test build #58910 has started](https://amplab.cs.berkeley.edu/jenkins/job/SparkPullRequestBuilder/58910/consoleFull)** for PR 13204 at commit [`db5ad62`](https://github.com/apache/spark/commit/db5ad6211da458bed0ac66f9c93d87ae5b87ef16).


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

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


[GitHub] spark pull request: [SPARK-15438] [SQL] improve explain of whole s...

Posted by SparkQA <gi...@git.apache.org>.
Github user SparkQA commented on the pull request:

    https://github.com/apache/spark/pull/13204#issuecomment-220532288
  
    **[Test build #3003 has started](https://amplab.cs.berkeley.edu/jenkins/job/NewSparkPullRequestBuilder/3003/consoleFull)** for PR 13204 at commit [`bc2aed4`](https://github.com/apache/spark/commit/bc2aed434157e3c3cf25f14520e52071dbceb635).


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

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


[GitHub] spark pull request: [SPARK-15438] [SQL] improve explain of whole s...

Posted by AmplabJenkins <gi...@git.apache.org>.
Github user AmplabJenkins commented on the pull request:

    https://github.com/apache/spark/pull/13204#issuecomment-220529370
  
    Test FAILed.
    Refer to this link for build results (access rights to CI server needed): 
    https://amplab.cs.berkeley.edu/jenkins//job/SparkPullRequestBuilder/58949/
    Test FAILed.


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

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


[GitHub] spark pull request: explain of whole stage codegen

Posted by yhuai <gi...@git.apache.org>.
Github user yhuai commented on the pull request:

    https://github.com/apache/spark/pull/13204#issuecomment-220525308
  
    I like 3. We can tell what operators are in a single WholeStageCodeGen operator and we can also know what are input operators of a WholeStageCodeGen.


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

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


[GitHub] spark pull request: explain of whole stage codegen

Posted by davies <gi...@git.apache.org>.
Github user davies commented on the pull request:

    https://github.com/apache/spark/pull/13204#issuecomment-220465870
  
    cc @marmbrus @rxin 


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

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


[GitHub] spark pull request: [SPARK-15438] [SQL] improve explain of whole s...

Posted by yucai <gi...@git.apache.org>.
Github user yucai commented on the pull request:

    https://github.com/apache/spark/pull/13204#issuecomment-222431501
  
    Dear Davies,
    
    Sometimes, it could be hard for me to recognize the boundary of whole stage codegen, 
    for example, in below SMJ, there are 3 separate whole stage codegen actually, can we read this info from the physical plan output?
    ```
       +- Exchange hashpartitioning(sourceIP#0, 200), None
          +- *TungstenAggregate(key=[sourceIP#0], ......
    	     +- *Project [pagerank#30,sourceIP#0,adRevenue#3]            
    			+- *SortMergeJoin [pageURL#29], [destURL#1], Inner, None
                   :- *Sort [pageURL#29 ASC], false, 0
                   :  +- Exchange hashpartitioning(pageURL#29, 200), None
                   :     +- ......
                   +- *Sort [destURL#1 ASC], false, 0
                      +- Exchange hashpartitioning(destURL#1, 200), None
                         +- ......
    ```


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

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


[GitHub] spark pull request: explain of whole stage codegen

Posted by davies <gi...@git.apache.org>.
Github user davies commented on the pull request:

    https://github.com/apache/spark/pull/13204#issuecomment-220505833
  
    @yucai Yes, corrected, thanks!


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

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


[GitHub] spark pull request: [SPARK-15438] [SQL] improve explain of whole stage codeg...

Posted by davies <gi...@git.apache.org>.
Github user davies commented on the pull request:

    https://github.com/apache/spark/pull/13204
  
    @yucai It's true that the case you posted is a little confusing, you can see the expected boundary on Spark UI. The other one is too verbose the see the plan (the logical parts), may not worth that.


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

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


[GitHub] spark pull request: [SPARK-15438] [SQL] improve explain of whole s...

Posted by SparkQA <gi...@git.apache.org>.
Github user SparkQA commented on the pull request:

    https://github.com/apache/spark/pull/13204#issuecomment-220672729
  
    **[Test build #3005 has started](https://amplab.cs.berkeley.edu/jenkins/job/NewSparkPullRequestBuilder/3005/consoleFull)** for PR 13204 at commit [`bc2aed4`](https://github.com/apache/spark/commit/bc2aed434157e3c3cf25f14520e52071dbceb635).


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

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


[GitHub] spark pull request: explain of whole stage codegen

Posted by AmplabJenkins <gi...@git.apache.org>.
Github user AmplabJenkins commented on the pull request:

    https://github.com/apache/spark/pull/13204#issuecomment-220480521
  
    Test PASSed.
    Refer to this link for build results (access rights to CI server needed): 
    https://amplab.cs.berkeley.edu/jenkins//job/SparkPullRequestBuilder/58910/
    Test PASSed.


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

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


[GitHub] spark pull request: explain of whole stage codegen

Posted by AmplabJenkins <gi...@git.apache.org>.
Github user AmplabJenkins commented on the pull request:

    https://github.com/apache/spark/pull/13204#issuecomment-220480514
  
    Merged build finished. Test PASSed.


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

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


[GitHub] spark pull request: explain of whole stage codegen

Posted by kiszk <gi...@git.apache.org>.
Github user kiszk commented on a diff in the pull request:

    https://github.com/apache/spark/pull/13204#discussion_r63987521
  
    --- Diff: sql/core/src/main/scala/org/apache/spark/sql/execution/WholeStageCodegenExec.scala ---
    @@ -247,9 +247,7 @@ case class InputAdapter(child: SparkPlan) extends UnaryExecNode with CodegenSupp
          """.stripMargin
       }
     
    -  override def simpleString: String = "INPUT"
    -
    -  override def treeChildren: Seq[SparkPlan] = Nil
    +  override def simpleString: String = "CodegenInput"
    --- End diff --
    
    Would it be better to synchronize name ```"CodegenInput``` in ```simpleString``` with the class name ```InputAdapter```?


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

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


[GitHub] spark pull request: [SPARK-15438] [SQL] improve explain of whole s...

Posted by rxin <gi...@git.apache.org>.
Github user rxin commented on the pull request:

    https://github.com/apache/spark/pull/13204#issuecomment-220705412
  
    New explain lgtm. I didn't look at the code though.



---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

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


[GitHub] spark pull request: [SPARK-15438] [SQL] improve explain of whole s...

Posted by rxin <gi...@git.apache.org>.
Github user rxin commented on the pull request:

    https://github.com/apache/spark/pull/13204#issuecomment-220709061
  
    Merging in master/2.0.



---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

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


[GitHub] spark pull request: [SPARK-15438] [SQL] improve explain of whole s...

Posted by davies <gi...@git.apache.org>.
Github user davies commented on the pull request:

    https://github.com/apache/spark/pull/13204#issuecomment-220527920
  
    @yhuai @marmbrus Updated to #3. 


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

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


[GitHub] spark pull request: [SPARK-15438] [SQL] improve explain of whole s...

Posted by asfgit <gi...@git.apache.org>.
Github user asfgit closed the pull request at:

    https://github.com/apache/spark/pull/13204


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

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


[GitHub] spark pull request: [SPARK-15438] [SQL] improve explain of whole s...

Posted by yhuai <gi...@git.apache.org>.
Github user yhuai commented on the pull request:

    https://github.com/apache/spark/pull/13204#issuecomment-220706346
  
    LGTM


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

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


[GitHub] spark pull request: [SPARK-15438] [SQL] improve explain of whole s...

Posted by AmplabJenkins <gi...@git.apache.org>.
Github user AmplabJenkins commented on the pull request:

    https://github.com/apache/spark/pull/13204#issuecomment-220529369
  
    Merged build finished. Test FAILed.


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

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


[GitHub] spark pull request: explain of whole stage codegen

Posted by SparkQA <gi...@git.apache.org>.
Github user SparkQA commented on the pull request:

    https://github.com/apache/spark/pull/13204#issuecomment-220480353
  
    **[Test build #58910 has finished](https://amplab.cs.berkeley.edu/jenkins/job/SparkPullRequestBuilder/58910/consoleFull)** for PR 13204 at commit [`db5ad62`](https://github.com/apache/spark/commit/db5ad6211da458bed0ac66f9c93d87ae5b87ef16).
     * This patch passes all tests.
     * This patch merges cleanly.
     * This patch adds no public classes.


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

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


[GitHub] spark pull request: [SPARK-15438] [SQL] improve explain of whole s...

Posted by SparkQA <gi...@git.apache.org>.
Github user SparkQA commented on the pull request:

    https://github.com/apache/spark/pull/13204#issuecomment-220529341
  
    **[Test build #58949 has finished](https://amplab.cs.berkeley.edu/jenkins/job/SparkPullRequestBuilder/58949/consoleFull)** for PR 13204 at commit [`bc2aed4`](https://github.com/apache/spark/commit/bc2aed434157e3c3cf25f14520e52071dbceb635).
     * This patch **fails MiMa tests**.
     * This patch merges cleanly.
     * This patch adds no public classes.


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

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


[GitHub] spark pull request: [SPARK-15438] [SQL] improve explain of whole s...

Posted by SparkQA <gi...@git.apache.org>.
Github user SparkQA commented on the pull request:

    https://github.com/apache/spark/pull/13204#issuecomment-220696314
  
    **[Test build #3005 has finished](https://amplab.cs.berkeley.edu/jenkins/job/NewSparkPullRequestBuilder/3005/consoleFull)** for PR 13204 at commit [`bc2aed4`](https://github.com/apache/spark/commit/bc2aed434157e3c3cf25f14520e52071dbceb635).
     * This patch passes all tests.
     * This patch merges cleanly.
     * This patch adds no public classes.


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

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


[GitHub] spark pull request: [SPARK-15438] [SQL] improve explain of whole s...

Posted by SparkQA <gi...@git.apache.org>.
Github user SparkQA commented on the pull request:

    https://github.com/apache/spark/pull/13204#issuecomment-220545057
  
    **[Test build #3003 has finished](https://amplab.cs.berkeley.edu/jenkins/job/NewSparkPullRequestBuilder/3003/consoleFull)** for PR 13204 at commit [`bc2aed4`](https://github.com/apache/spark/commit/bc2aed434157e3c3cf25f14520e52071dbceb635).
     * This patch **fails PySpark unit tests**.
     * This patch merges cleanly.
     * This patch adds no public classes.


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

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


[GitHub] spark pull request: [SPARK-15438] [SQL] improve explain of whole s...

Posted by SparkQA <gi...@git.apache.org>.
Github user SparkQA commented on the pull request:

    https://github.com/apache/spark/pull/13204#issuecomment-220527528
  
    **[Test build #58949 has started](https://amplab.cs.berkeley.edu/jenkins/job/SparkPullRequestBuilder/58949/consoleFull)** for PR 13204 at commit [`bc2aed4`](https://github.com/apache/spark/commit/bc2aed434157e3c3cf25f14520e52071dbceb635).


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

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


[GitHub] spark pull request: explain of whole stage codegen

Posted by davies <gi...@git.apache.org>.
Github user davies commented on the pull request:

    https://github.com/apache/spark/pull/13204#issuecomment-220466852
  
    Another proposal is to have a special prefix for the operators that are part of whole stage codegen.
    ```
    >>> df = sqlCtx.range(1000);df2 = sqlCtx.range(1000);df.join(pyspark.sql.functions.broadcast(df2), 'id').explain()
    == Physical Plan ==
    *Project [id#0L]
    +- *BroadcastHashJoin [id#0L], [id#3L], Inner, BuildRight, None
        :- *Range 0, 1, 4, 1000, [id#0L]
        +- BroadcastExchange HashedRelationBroadcastMode(List(input[0, bigint, false]))
            +- *Range 0, 1, 4, 1000, [id#3L]
    ```


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

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