carbondata git commit: [CARBONDATA-2317] Concurrent datamap with same name and schema creation throws exception
Repository: carbondata Updated Branches: refs/heads/branch-1.3 3c48df396 -> cc2a74e65 [CARBONDATA-2317] Concurrent datamap with same name and schema creation throws exception Concurrent datamap with same name and schema creation throws exception This closes #2156 Project: http://git-wip-us.apache.org/repos/asf/carbondata/repo Commit: http://git-wip-us.apache.org/repos/asf/carbondata/commit/cc2a74e6 Tree: http://git-wip-us.apache.org/repos/asf/carbondata/tree/cc2a74e6 Diff: http://git-wip-us.apache.org/repos/asf/carbondata/diff/cc2a74e6 Branch: refs/heads/branch-1.3 Commit: cc2a74e65001ee73da4fad47852984d4f1c42adf Parents: 3c48df3 Author: rahulforallpAuthored: Fri Apr 6 15:17:54 2018 +0530 Committer: manishgupta88 Committed: Wed Apr 11 13:55:41 2018 +0530 -- .../preaggregate/TestPreAggCreateCommand.scala | 43 .../table/CarbonCreateTableCommand.scala| 27 ++-- 2 files changed, 58 insertions(+), 12 deletions(-) -- http://git-wip-us.apache.org/repos/asf/carbondata/blob/cc2a74e6/integration/spark-common-test/src/test/scala/org/apache/carbondata/integration/spark/testsuite/preaggregate/TestPreAggCreateCommand.scala -- diff --git a/integration/spark-common-test/src/test/scala/org/apache/carbondata/integration/spark/testsuite/preaggregate/TestPreAggCreateCommand.scala b/integration/spark-common-test/src/test/scala/org/apache/carbondata/integration/spark/testsuite/preaggregate/TestPreAggCreateCommand.scala index 5b7c310..5813211 100644 --- a/integration/spark-common-test/src/test/scala/org/apache/carbondata/integration/spark/testsuite/preaggregate/TestPreAggCreateCommand.scala +++ b/integration/spark-common-test/src/test/scala/org/apache/carbondata/integration/spark/testsuite/preaggregate/TestPreAggCreateCommand.scala @@ -17,7 +17,14 @@ package org.apache.carbondata.integration.spark.testsuite.preaggregate +import java.util +import java.util.concurrent.{Callable, ExecutorService, Executors, TimeUnit} + import scala.collection.JavaConverters._ +import scala.concurrent.ExecutionContext.Implicits.global +import scala.concurrent.{Await, Future} +import scala.concurrent.duration.Duration +import scala.util.{Failure, Success} import org.apache.spark.sql.{AnalysisException, CarbonDatasourceHadoopRelation, Row} import org.apache.spark.sql.catalyst.plans.logical.LogicalPlan @@ -415,6 +422,42 @@ class TestPreAggCreateCommand extends QueryTest with BeforeAndAfterAll { sql("use default") } + test("test creation of multiple preaggregate of same name concurrently ") { +sql("DROP TABLE IF EXISTS tbl_concurr") +sql( + "create table if not exists tbl_concurr(imei string,age int,mac string ,prodate timestamp," + + "update timestamp,gamepoint double,contrid double) stored by 'carbondata' ") + +var executorService: ExecutorService = Executors.newCachedThreadPool() +val tasks = new util.ArrayList[Callable[String]]() +var i = 0 +val count = 5 +while (i < count) { + tasks +.add(new QueryTask( + s"""create datamap agg_concu1 on table tbl_concurr using + |'preaggregate' as select prodate, mac from tbl_concurr group by prodate,mac""" +.stripMargin)) + i = i + 1 +} +executorService.invokeAll(tasks) + +checkExistence(sql("show tables"), true, "agg_concu1", "tbl_concurr") +executorService.shutdown() + } + + class QueryTask(query: String) extends Callable[String] { +override def call(): String = { + var result = "SUCCESS" + try { +sql(query).collect() + } catch { +case exception: Exception => LOGGER.error(exception.getMessage) + } + result +} + } + def getCarbontable(plan: LogicalPlan) : CarbonTable ={ var carbonTable : CarbonTable = null plan.transform { http://git-wip-us.apache.org/repos/asf/carbondata/blob/cc2a74e6/integration/spark2/src/main/scala/org/apache/spark/sql/execution/command/table/CarbonCreateTableCommand.scala -- diff --git a/integration/spark2/src/main/scala/org/apache/spark/sql/execution/command/table/CarbonCreateTableCommand.scala b/integration/spark2/src/main/scala/org/apache/spark/sql/execution/command/table/CarbonCreateTableCommand.scala index 22dab27..e05cda6 100644 --- a/integration/spark2/src/main/scala/org/apache/spark/sql/execution/command/table/CarbonCreateTableCommand.scala +++ b/integration/spark2/src/main/scala/org/apache/spark/sql/execution/command/table/CarbonCreateTableCommand.scala @@ -119,19 +119,22 @@ case class CarbonCreateTableCommand( // isVisible property is added to hive table properties to differentiate between
carbondata git commit: [CARBONDATA-2317] Concurrent datamap with same name and schema creation throws exception
Repository: carbondata Updated Branches: refs/heads/master 9ca9b6d0c -> 4cbd5cdf2 [CARBONDATA-2317] Concurrent datamap with same name and schema creation throws exception Concurrent datamap with same name and schema creation throws exception This closes #2143 Project: http://git-wip-us.apache.org/repos/asf/carbondata/repo Commit: http://git-wip-us.apache.org/repos/asf/carbondata/commit/4cbd5cdf Tree: http://git-wip-us.apache.org/repos/asf/carbondata/tree/4cbd5cdf Diff: http://git-wip-us.apache.org/repos/asf/carbondata/diff/4cbd5cdf Branch: refs/heads/master Commit: 4cbd5cdf20341f1ae07c7cbb8275c1e333856cae Parents: 9ca9b6d Author: rahulforallpAuthored: Fri Apr 6 15:17:54 2018 +0530 Committer: manishgupta88 Committed: Wed Apr 11 11:15:15 2018 +0530 -- .../preaggregate/TestPreAggCreateCommand.scala | 44 .../table/CarbonCreateTableCommand.scala| 31 +++--- 2 files changed, 61 insertions(+), 14 deletions(-) -- http://git-wip-us.apache.org/repos/asf/carbondata/blob/4cbd5cdf/integration/spark-common-test/src/test/scala/org/apache/carbondata/integration/spark/testsuite/preaggregate/TestPreAggCreateCommand.scala -- diff --git a/integration/spark-common-test/src/test/scala/org/apache/carbondata/integration/spark/testsuite/preaggregate/TestPreAggCreateCommand.scala b/integration/spark-common-test/src/test/scala/org/apache/carbondata/integration/spark/testsuite/preaggregate/TestPreAggCreateCommand.scala index e546fe8..7cb1adf 100644 --- a/integration/spark-common-test/src/test/scala/org/apache/carbondata/integration/spark/testsuite/preaggregate/TestPreAggCreateCommand.scala +++ b/integration/spark-common-test/src/test/scala/org/apache/carbondata/integration/spark/testsuite/preaggregate/TestPreAggCreateCommand.scala @@ -17,7 +17,14 @@ package org.apache.carbondata.integration.spark.testsuite.preaggregate +import java.util +import java.util.concurrent.{Callable, ExecutorService, Executors, TimeUnit} + import scala.collection.JavaConverters._ +import scala.concurrent.ExecutionContext.Implicits.global +import scala.concurrent.{Await, Future} +import scala.concurrent.duration.Duration +import scala.util.{Failure, Success} import org.apache.spark.sql.{AnalysisException, CarbonDatasourceHadoopRelation, Row} import org.apache.spark.sql.catalyst.plans.logical.LogicalPlan @@ -429,6 +436,43 @@ class TestPreAggCreateCommand extends QueryTest with BeforeAndAfterAll { } } + test("test creation of multiple preaggregate of same name concurrently ") { +sql("DROP TABLE IF EXISTS tbl_concurr") +sql( + "create table if not exists tbl_concurr(imei string,age int,mac string ,prodate timestamp," + + "update timestamp,gamepoint double,contrid double) stored by 'carbondata' ") + +var executorService: ExecutorService = Executors.newCachedThreadPool() +val tasks = new util.ArrayList[Callable[String]]() +var i = 0 +val count = 5 +while (i < count) { + tasks +.add(new QueryTask( + s"""create datamap agg_concu1 on table tbl_concurr using + |'preaggregate' as select prodate, mac from tbl_concurr group by prodate,mac""" +.stripMargin)) + i = i + 1 +} +executorService.invokeAll(tasks) + +checkExistence(sql("show tables"), true, "agg_concu1", "tbl_concurr") +executorService.shutdown() + } + + class QueryTask(query: String) extends Callable[String] { +override def call(): String = { + var result = "SUCCESS" + try { +sql(query).collect() + } catch { +case exception: Exception => LOGGER.error(exception.getMessage) + } + result +} + } + + def getCarbonTable(plan: LogicalPlan) : CarbonTable ={ var carbonTable : CarbonTable = null plan.transform { http://git-wip-us.apache.org/repos/asf/carbondata/blob/4cbd5cdf/integration/spark2/src/main/scala/org/apache/spark/sql/execution/command/table/CarbonCreateTableCommand.scala -- diff --git a/integration/spark2/src/main/scala/org/apache/spark/sql/execution/command/table/CarbonCreateTableCommand.scala b/integration/spark2/src/main/scala/org/apache/spark/sql/execution/command/table/CarbonCreateTableCommand.scala index 65c6269..6266c53 100644 --- a/integration/spark2/src/main/scala/org/apache/spark/sql/execution/command/table/CarbonCreateTableCommand.scala +++ b/integration/spark2/src/main/scala/org/apache/spark/sql/execution/command/table/CarbonCreateTableCommand.scala @@ -121,21 +121,24 @@ case class CarbonCreateTableCommand( // isVisible property is added to hive table properties to differentiate between main //