Uploaded image for project: 'Spark'
  1. Spark
  2. SPARK-5351

Can't zip RDDs with unequal numbers of partitions in ReplicatedVertexView.upgrade()

    XMLWordPrintableJSON

Details

    • Bug
    • Status: Resolved
    • Major
    • Resolution: Fixed
    • None
    • 1.2.1, 1.3.0
    • GraphX
    • None

    Description

      If the value of 'spark.default.parallelism' does not match the number of partitoins in EdgePartition(EdgeRDDImpl),
      the following error occurs in ReplicatedVertexView.scala:72;

      object GraphTest extends Logging {
      def run[VD: ClassTag, ED: ClassTag](graph: Graph[VD, ED]): VertexRDD[Int] = {
      graph.aggregateMessages[Int](
      ctx => {
      ctx.sendToSrc(1)
      ctx.sendToDst(2)
      },
      _ + _)
      }
      }

      val g = GraphLoader.edgeListFile(sc, "graph.txt")
      val rdd = GraphTest.run(g)

      java.lang.IllegalArgumentException: Can't zip RDDs with unequal numbers of partitions
      at org.apache.spark.rdd.ZippedPartitionsBaseRDD.getPartitions(ZippedPartitionsRDD.scala:57)
      at org.apache.spark.rdd.RDD$$anonfun$partitions$2.apply(RDD.scala:206)
      at org.apache.spark.rdd.RDD$$anonfun$partitions$2.apply(RDD.scala:204)
      at scala.Option.getOrElse(Option.scala:120)
      at org.apache.spark.rdd.RDD.partitions(RDD.scala:204)
      at org.apache.spark.rdd.MapPartitionsRDD.getPartitions(MapPartitionsRDD.scala:32)
      at org.apache.spark.rdd.RDD$$anonfun$partitions$2.apply(RDD.scala:206)
      at org.apache.spark.rdd.RDD$$anonfun$partitions$2.apply(RDD.scala:204)
      at scala.Option.getOrElse(Option.scala:120)
      at org.apache.spark.rdd.RDD.partitions(RDD.scala:204)
      at org.apache.spark.ShuffleDependency.<init>(Dependency.scala:82)
      at org.apache.spark.rdd.ShuffledRDD.getDependencies(ShuffledRDD.scala:80)
      at org.apache.spark.rdd.RDD$$anonfun$dependencies$2.apply(RDD.scala:193)
      at org.apache.spark.rdd.RDD$$anonfun$dependencies$2.apply(RDD.scala:191)
      ...

      Attachments

        Issue Links

          Activity

            People

              Unassigned Unassigned
              maropu Takeshi Yamamuro
              Votes:
              0 Vote for this issue
              Watchers:
              3 Start watching this issue

              Dates

                Created:
                Updated:
                Resolved: