You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@carbondata.apache.org by ja...@apache.org on 2018/02/27 16:38:56 UTC
[10/50] carbondata git commit: [CARBONDATA-2025] Unify all path
construction through CarbonTablePath static method
http://git-wip-us.apache.org/repos/asf/carbondata/blob/2e1f2b95/core/src/test/java/org/apache/carbondata/core/util/path/CarbonFormatDirectoryStructureTest.java
----------------------------------------------------------------------
diff --git a/core/src/test/java/org/apache/carbondata/core/util/path/CarbonFormatDirectoryStructureTest.java b/core/src/test/java/org/apache/carbondata/core/util/path/CarbonFormatDirectoryStructureTest.java
index a1ccab3..4293536 100644
--- a/core/src/test/java/org/apache/carbondata/core/util/path/CarbonFormatDirectoryStructureTest.java
+++ b/core/src/test/java/org/apache/carbondata/core/util/path/CarbonFormatDirectoryStructureTest.java
@@ -39,21 +39,19 @@ public class CarbonFormatDirectoryStructureTest {
*/
@Test public void testTablePathStructure() throws IOException {
CarbonTableIdentifier tableIdentifier = new CarbonTableIdentifier("d1", "t1", UUID.randomUUID().toString());
- CarbonStorePath carbonStorePath = new CarbonStorePath(CARBON_STORE);
- AbsoluteTableIdentifier absoluteTableIdentifier =
+ AbsoluteTableIdentifier identifier =
AbsoluteTableIdentifier.from(CARBON_STORE + "/d1/t1", tableIdentifier);
- CarbonTablePath carbonTablePath = CarbonStorePath.getCarbonTablePath(absoluteTableIdentifier);
- assertTrue(carbonTablePath.getPath().replace("\\", "/").equals(CARBON_STORE + "/d1/t1"));
- assertTrue(carbonTablePath.getSchemaFilePath().replace("\\", "/").equals(CARBON_STORE + "/d1/t1/Metadata/schema"));
- assertTrue(carbonTablePath.getTableStatusFilePath().replace("\\", "/")
+ assertTrue(identifier.getTablePath().replace("\\", "/").equals(CARBON_STORE + "/d1/t1"));
+ assertTrue(CarbonTablePath.getSchemaFilePath(identifier.getTablePath()).replace("\\", "/").equals(CARBON_STORE + "/d1/t1/Metadata/schema"));
+ assertTrue(CarbonTablePath.getTableStatusFilePath(identifier.getTablePath()).replace("\\", "/")
.equals(CARBON_STORE + "/d1/t1/Metadata/tablestatus"));
- assertTrue(carbonTablePath.getDictionaryFilePath("t1_c1").replace("\\", "/")
+ assertTrue(CarbonTablePath.getDictionaryFilePath(identifier.getTablePath(), "t1_c1").replace("\\", "/")
.equals(CARBON_STORE + "/d1/t1/Metadata/t1_c1.dict"));
- assertTrue(carbonTablePath.getDictionaryMetaFilePath("t1_c1").replace("\\", "/")
+ assertTrue(CarbonTablePath.getDictionaryMetaFilePath(identifier.getTablePath(), "t1_c1").replace("\\", "/")
.equals(CARBON_STORE + "/d1/t1/Metadata/t1_c1.dictmeta"));
- assertTrue(carbonTablePath.getSortIndexFilePath("t1_c1").replace("\\", "/")
+ assertTrue(CarbonTablePath.getSortIndexFilePath(identifier.getTablePath(),"t1_c1").replace("\\", "/")
.equals(CARBON_STORE + "/d1/t1/Metadata/t1_c1.sortindex"));
- assertTrue(carbonTablePath.getCarbonDataFilePath("2", 3, 4L, 0, 0, "999").replace("\\", "/")
+ assertTrue(CarbonTablePath.getCarbonDataFilePath(identifier.getTablePath(), "2", 3, 4L, 0, 0, "999").replace("\\", "/")
.equals(CARBON_STORE + "/d1/t1/Fact/Part0/Segment_2/part-3-4_batchno0-0-999.carbondata"));
}
http://git-wip-us.apache.org/repos/asf/carbondata/blob/2e1f2b95/core/src/test/java/org/apache/carbondata/core/util/path/CarbonFormatSharedDictionaryTest.java
----------------------------------------------------------------------
diff --git a/core/src/test/java/org/apache/carbondata/core/util/path/CarbonFormatSharedDictionaryTest.java b/core/src/test/java/org/apache/carbondata/core/util/path/CarbonFormatSharedDictionaryTest.java
deleted file mode 100644
index 91384c1..0000000
--- a/core/src/test/java/org/apache/carbondata/core/util/path/CarbonFormatSharedDictionaryTest.java
+++ /dev/null
@@ -1,44 +0,0 @@
-/*
- * 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.carbondata.core.util.path;
-
-import java.io.IOException;
-
-import org.junit.Test;
-
-import static junit.framework.TestCase.assertTrue;
-
-/**
- * test shared dictionary paths
- */
-public class CarbonFormatSharedDictionaryTest {
-
- private final String CARBON_STORE = "/opt/carbonstore";
-
- /**
- * test shared dictionary location
- */
- @Test public void testSharedDimentionLocation() throws IOException {
- assertTrue(CarbonSharedDictionaryPath.getDictionaryFilePath(CARBON_STORE, "d1", "shared_c1").replace("\\", "/")
- .equals(CARBON_STORE + "/d1/SharedDictionary/shared_c1.dict"));
- assertTrue(CarbonSharedDictionaryPath.getDictionaryMetaFilePath(CARBON_STORE, "d1", "shared_c1").replace("\\", "/")
- .equals(CARBON_STORE + "/d1/SharedDictionary/shared_c1.dictmeta"));
- assertTrue(CarbonSharedDictionaryPath.getSortIndexFilePath(CARBON_STORE, "d1", "shared_c1").replace("\\", "/")
- .equals(CARBON_STORE + "/d1/SharedDictionary/shared_c1.sortindex"));
- }
-}
http://git-wip-us.apache.org/repos/asf/carbondata/blob/2e1f2b95/core/src/test/java/org/apache/carbondata/core/writer/CarbonDictionaryWriterImplTest.java
----------------------------------------------------------------------
diff --git a/core/src/test/java/org/apache/carbondata/core/writer/CarbonDictionaryWriterImplTest.java b/core/src/test/java/org/apache/carbondata/core/writer/CarbonDictionaryWriterImplTest.java
index 308d041..d5500e1 100644
--- a/core/src/test/java/org/apache/carbondata/core/writer/CarbonDictionaryWriterImplTest.java
+++ b/core/src/test/java/org/apache/carbondata/core/writer/CarbonDictionaryWriterImplTest.java
@@ -44,7 +44,6 @@ import org.apache.carbondata.core.reader.CarbonDictionaryMetadataReaderImpl;
import org.apache.carbondata.core.reader.CarbonDictionaryReaderImpl;
import org.apache.carbondata.core.util.CarbonProperties;
import org.apache.carbondata.core.util.CarbonUtil;
-import org.apache.carbondata.core.util.path.CarbonStorePath;
import org.apache.carbondata.core.util.path.CarbonTablePath;
import org.apache.carbondata.format.ColumnDictionaryChunkMeta;
@@ -72,7 +71,7 @@ public class CarbonDictionaryWriterImplTest {
private String tableName;
- private String carbonStorePath;
+ private String tablePath;
private ColumnIdentifier columnIdentifier;
@@ -100,10 +99,10 @@ public class CarbonDictionaryWriterImplTest {
init();
this.databaseName = props.getProperty("database", "testSchema");
this.tableName = props.getProperty("tableName", "carbon");
- this.carbonStorePath = props.getProperty("storePath", "carbonStore");
+ this.tablePath = props.getProperty("storePath", "carbonStore");
this.columnIdentifier = new ColumnIdentifier("Name", null, null);
carbonTableIdentifier = new CarbonTableIdentifier(databaseName, tableName, UUID.randomUUID().toString());
- absoluteTableIdentifier = AbsoluteTableIdentifier.from(carbonStorePath, carbonTableIdentifier);
+ absoluteTableIdentifier = AbsoluteTableIdentifier.from(tablePath, carbonTableIdentifier);
this.dictionaryColumnUniqueIdentifier =
new DictionaryColumnUniqueIdentifier(absoluteTableIdentifier, columnIdentifier,
columnIdentifier.getDataType());
@@ -479,8 +478,8 @@ public class CarbonDictionaryWriterImplTest {
* this method will delete the store path
*/
private void deleteStorePath() {
- FileFactory.FileType fileType = FileFactory.getFileType(this.carbonStorePath);
- CarbonFile carbonFile = FileFactory.getCarbonFile(this.carbonStorePath, fileType);
+ FileFactory.FileType fileType = FileFactory.getFileType(this.tablePath);
+ CarbonFile carbonFile = FileFactory.getCarbonFile(this.tablePath, fileType);
deleteRecursiveSilent(carbonFile);
}
@@ -528,14 +527,12 @@ public class CarbonDictionaryWriterImplTest {
* this method will form the dictionary directory paths
*/
private void initDictionaryDirPaths() throws IOException {
- CarbonTablePath carbonTablePath =
- CarbonStorePath.getCarbonTablePath(this.carbonStorePath, carbonTableIdentifier);
- String dictionaryLocation = carbonTablePath.getMetadataDirectoryPath();
+ String dictionaryLocation = CarbonTablePath.getMetadataPath(tablePath);
FileFactory.FileType fileType = FileFactory.getFileType(dictionaryLocation);
if(!FileFactory.isFileExist(dictionaryLocation, fileType)) {
FileFactory.mkdirs(dictionaryLocation, fileType);
}
- this.dictionaryFilePath = carbonTablePath.getDictionaryFilePath(columnIdentifier.getColumnId());
- this.dictionaryMetaFilePath = carbonTablePath.getDictionaryMetaFilePath(columnIdentifier.getColumnId());
+ this.dictionaryFilePath = CarbonTablePath.getDictionaryFilePath(tablePath, columnIdentifier.getColumnId());
+ this.dictionaryMetaFilePath = CarbonTablePath.getDictionaryMetaFilePath(tablePath, columnIdentifier.getColumnId());
}
}
http://git-wip-us.apache.org/repos/asf/carbondata/blob/2e1f2b95/examples/spark2/src/main/scala/org/apache/carbondata/examples/CarbonBatchSparkStreamingExample.scala
----------------------------------------------------------------------
diff --git a/examples/spark2/src/main/scala/org/apache/carbondata/examples/CarbonBatchSparkStreamingExample.scala b/examples/spark2/src/main/scala/org/apache/carbondata/examples/CarbonBatchSparkStreamingExample.scala
index 6ae87b9..40460c6 100644
--- a/examples/spark2/src/main/scala/org/apache/carbondata/examples/CarbonBatchSparkStreamingExample.scala
+++ b/examples/spark2/src/main/scala/org/apache/carbondata/examples/CarbonBatchSparkStreamingExample.scala
@@ -24,10 +24,6 @@ import org.apache.spark.rdd.RDD
import org.apache.spark.sql.{CarbonEnv, SaveMode, SparkSession}
import org.apache.spark.streaming.{Seconds, StreamingContext, Time}
-import org.apache.carbondata.core.constants.CarbonCommonConstants
-import org.apache.carbondata.core.util.CarbonProperties
-import org.apache.carbondata.core.util.path.{CarbonStorePath, CarbonTablePath}
-
/**
* This example introduces how to use CarbonData batch load to integrate
* with Spark Streaming(it's DStream, not Spark Structured Streaming)
@@ -74,7 +70,6 @@ object CarbonBatchSparkStreamingExample {
| """.stripMargin)
val carbonTable = CarbonEnv.getCarbonTable(Some("default"), streamTableName)(spark)
- val tablePath = CarbonStorePath.getCarbonTablePath(carbonTable.getAbsoluteTableIdentifier)
// batch load
val path = s"$rootPath/examples/spark2/src/main/resources/streamSample.csv"
spark.sql(
@@ -88,7 +83,7 @@ object CarbonBatchSparkStreamingExample {
val serverSocket = new ServerSocket(7071)
val thread1 = writeSocket(serverSocket)
val thread2 = showTableCount(spark, streamTableName)
- val ssc = startStreaming(spark, streamTableName, tablePath, checkpointPath)
+ val ssc = startStreaming(spark, streamTableName, checkpointPath)
// wait for stop signal to stop Spark Streaming App
waitForStopSignal(ssc)
// it need to start Spark Streaming App in main thread
@@ -153,7 +148,7 @@ object CarbonBatchSparkStreamingExample {
}
def startStreaming(spark: SparkSession, tableName: String,
- tablePath: CarbonTablePath, checkpointPath: String): StreamingContext = {
+ checkpointPath: String): StreamingContext = {
var ssc: StreamingContext = null
try {
// recommend: the batch interval must set larger, such as 30s, 1min.
http://git-wip-us.apache.org/repos/asf/carbondata/blob/2e1f2b95/examples/spark2/src/main/scala/org/apache/carbondata/examples/CarbonStructuredStreamingExample.scala
----------------------------------------------------------------------
diff --git a/examples/spark2/src/main/scala/org/apache/carbondata/examples/CarbonStructuredStreamingExample.scala b/examples/spark2/src/main/scala/org/apache/carbondata/examples/CarbonStructuredStreamingExample.scala
index 247a59b..bca7fa3 100644
--- a/examples/spark2/src/main/scala/org/apache/carbondata/examples/CarbonStructuredStreamingExample.scala
+++ b/examples/spark2/src/main/scala/org/apache/carbondata/examples/CarbonStructuredStreamingExample.scala
@@ -24,8 +24,9 @@ import org.apache.spark.sql.{CarbonEnv, SparkSession}
import org.apache.spark.sql.streaming.{ProcessingTime, StreamingQuery}
import org.apache.carbondata.core.constants.CarbonCommonConstants
+import org.apache.carbondata.core.metadata.schema.table.CarbonTable
import org.apache.carbondata.core.util.CarbonProperties
-import org.apache.carbondata.core.util.path.{CarbonStorePath, CarbonTablePath}
+import org.apache.carbondata.core.util.path.CarbonTablePath
// scalastyle:off println
object CarbonStructuredStreamingExample {
@@ -89,7 +90,6 @@ object CarbonStructuredStreamingExample {
}
val carbonTable = CarbonEnv.getCarbonTable(Some("default"), streamTableName)(spark)
- val tablePath = CarbonStorePath.getCarbonTablePath(carbonTable.getAbsoluteTableIdentifier)
// batch load
val path = s"$rootPath/examples/spark2/src/main/resources/streamSample.csv"
spark.sql(
@@ -101,7 +101,7 @@ object CarbonStructuredStreamingExample {
// streaming ingest
val serverSocket = new ServerSocket(7071)
- val thread1 = startStreaming(spark, tablePath)
+ val thread1 = startStreaming(spark, carbonTable)
val thread2 = writeSocket(serverSocket)
val thread3 = showTableCount(spark, streamTableName)
@@ -152,7 +152,7 @@ object CarbonStructuredStreamingExample {
thread
}
- def startStreaming(spark: SparkSession, tablePath: CarbonTablePath): Thread = {
+ def startStreaming(spark: SparkSession, carbonTable: CarbonTable): Thread = {
val thread = new Thread() {
override def run(): Unit = {
var qry: StreamingQuery = null
@@ -167,7 +167,8 @@ object CarbonStructuredStreamingExample {
qry = readSocketDF.writeStream
.format("carbondata")
.trigger(ProcessingTime("5 seconds"))
- .option("checkpointLocation", tablePath.getStreamingCheckpointDir)
+ .option("checkpointLocation",
+ CarbonTablePath.getStreamingCheckpointDir(carbonTable.getTablePath))
.option("dbName", "default")
.option("tableName", "stream_table")
.start()
http://git-wip-us.apache.org/repos/asf/carbondata/blob/2e1f2b95/hadoop/src/main/java/org/apache/carbondata/hadoop/api/CarbonTableInputFormat.java
----------------------------------------------------------------------
diff --git a/hadoop/src/main/java/org/apache/carbondata/hadoop/api/CarbonTableInputFormat.java b/hadoop/src/main/java/org/apache/carbondata/hadoop/api/CarbonTableInputFormat.java
index 087cf55..9dc8d38 100644
--- a/hadoop/src/main/java/org/apache/carbondata/hadoop/api/CarbonTableInputFormat.java
+++ b/hadoop/src/main/java/org/apache/carbondata/hadoop/api/CarbonTableInputFormat.java
@@ -70,7 +70,6 @@ import org.apache.carbondata.core.util.CarbonTimeStatisticsFactory;
import org.apache.carbondata.core.util.CarbonUtil;
import org.apache.carbondata.core.util.DataTypeConverter;
import org.apache.carbondata.core.util.DataTypeConverterImpl;
-import org.apache.carbondata.core.util.path.CarbonStorePath;
import org.apache.carbondata.core.util.path.CarbonTablePath;
import org.apache.carbondata.format.BlockIndex;
import org.apache.carbondata.hadoop.CarbonInputSplit;
@@ -249,10 +248,6 @@ public class CarbonTableInputFormat<T> extends FileInputFormat<Void, T> {
}
}
- private static CarbonTablePath getTablePath(AbsoluteTableIdentifier absIdentifier) {
- return CarbonStorePath.getCarbonTablePath(absIdentifier);
- }
-
/**
* Set list of segments to access
*/
@@ -495,11 +490,10 @@ public class CarbonTableInputFormat<T> extends FileInputFormat<Void, T> {
List<InputSplit> splits = new ArrayList<InputSplit>();
if (streamSegments != null && !streamSegments.isEmpty()) {
- CarbonTablePath tablePath = CarbonStorePath.getCarbonTablePath(identifier);
long minSize = Math.max(getFormatMinSplitSize(), getMinSplitSize(job));
long maxSize = getMaxSplitSize(job);
for (Segment segment : streamSegments) {
- String segmentDir = tablePath.getSegmentDir(segment.getSegmentNo());
+ String segmentDir = CarbonTablePath.getSegmentPath(identifier.getTablePath(), segment.getSegmentNo());
FileFactory.FileType fileType = FileFactory.getFileType(segmentDir);
if (FileFactory.isFileExist(segmentDir, fileType)) {
String indexName = CarbonTablePath.getCarbonStreamIndexFileName();
http://git-wip-us.apache.org/repos/asf/carbondata/blob/2e1f2b95/hadoop/src/main/java/org/apache/carbondata/hadoop/streaming/CarbonStreamRecordWriter.java
----------------------------------------------------------------------
diff --git a/hadoop/src/main/java/org/apache/carbondata/hadoop/streaming/CarbonStreamRecordWriter.java b/hadoop/src/main/java/org/apache/carbondata/hadoop/streaming/CarbonStreamRecordWriter.java
index 3ef8afc..a4b3be8 100644
--- a/hadoop/src/main/java/org/apache/carbondata/hadoop/streaming/CarbonStreamRecordWriter.java
+++ b/hadoop/src/main/java/org/apache/carbondata/hadoop/streaming/CarbonStreamRecordWriter.java
@@ -37,7 +37,6 @@ import org.apache.carbondata.core.metadata.schema.table.column.ColumnSchema;
import org.apache.carbondata.core.util.CarbonMetadataUtil;
import org.apache.carbondata.core.util.CarbonUtil;
import org.apache.carbondata.core.util.DataTypeUtil;
-import org.apache.carbondata.core.util.path.CarbonStorePath;
import org.apache.carbondata.core.util.path.CarbonTablePath;
import org.apache.carbondata.format.FileHeader;
import org.apache.carbondata.processing.loading.BadRecordsLogger;
@@ -127,9 +126,8 @@ public class CarbonStreamRecordWriter extends RecordWriter<Void, Object> {
maxCacheSize = hadoopConf.getInt(CarbonStreamOutputFormat.CARBON_STREAM_CACHE_SIZE,
CarbonStreamOutputFormat.CARBON_STREAM_CACHE_SIZE_DEFAULT);
- CarbonTablePath tablePath =
- CarbonStorePath.getCarbonTablePath(carbonTable.getAbsoluteTableIdentifier());
- segmentDir = tablePath.getSegmentDir(segmentId);
+ segmentDir = CarbonTablePath.getSegmentPath(
+ carbonTable.getAbsoluteTableIdentifier().getTablePath(), segmentId);
fileName = CarbonTablePath.getCarbonDataFileName(0, taskNo, 0, 0, "0");
}
http://git-wip-us.apache.org/repos/asf/carbondata/blob/2e1f2b95/hadoop/src/main/java/org/apache/carbondata/hadoop/util/SchemaReader.java
----------------------------------------------------------------------
diff --git a/hadoop/src/main/java/org/apache/carbondata/hadoop/util/SchemaReader.java b/hadoop/src/main/java/org/apache/carbondata/hadoop/util/SchemaReader.java
index 423bb2a..dfa8dd1 100644
--- a/hadoop/src/main/java/org/apache/carbondata/hadoop/util/SchemaReader.java
+++ b/hadoop/src/main/java/org/apache/carbondata/hadoop/util/SchemaReader.java
@@ -27,7 +27,7 @@ import org.apache.carbondata.core.metadata.converter.ThriftWrapperSchemaConverte
import org.apache.carbondata.core.metadata.schema.table.CarbonTable;
import org.apache.carbondata.core.metadata.schema.table.TableInfo;
import org.apache.carbondata.core.util.CarbonUtil;
-import org.apache.carbondata.core.util.path.CarbonStorePath;
+import org.apache.carbondata.core.util.path.CarbonTablePath;
import org.apache.carbondata.core.util.path.CarbonTablePath;
/**
@@ -37,8 +37,7 @@ public class SchemaReader {
public static CarbonTable readCarbonTableFromStore(AbsoluteTableIdentifier identifier)
throws IOException {
- CarbonTablePath carbonTablePath = CarbonStorePath.getCarbonTablePath(identifier);
- String schemaFilePath = carbonTablePath.getSchemaFilePath();
+ String schemaFilePath = CarbonTablePath.getSchemaFilePath(identifier.getTablePath());
if (FileFactory.isFileExist(schemaFilePath, FileFactory.FileType.LOCAL) ||
FileFactory.isFileExist(schemaFilePath, FileFactory.FileType.HDFS) ||
FileFactory.isFileExist(schemaFilePath, FileFactory.FileType.S3) ||
@@ -46,7 +45,7 @@ public class SchemaReader {
String tableName = identifier.getCarbonTableIdentifier().getTableName();
org.apache.carbondata.format.TableInfo tableInfo =
- CarbonUtil.readSchemaFile(carbonTablePath.getSchemaFilePath());
+ CarbonUtil.readSchemaFile(CarbonTablePath.getSchemaFilePath(identifier.getTablePath()));
SchemaConverter schemaConverter = new ThriftWrapperSchemaConverterImpl();
TableInfo wrapperTableInfo = schemaConverter.fromExternalToWrapperTableInfo(
tableInfo,
@@ -63,22 +62,21 @@ public class SchemaReader {
/**
* the method returns the Wrapper TableInfo
*
- * @param absoluteTableIdentifier
+ * @param identifier
* @return
*/
- public static TableInfo getTableInfo(AbsoluteTableIdentifier absoluteTableIdentifier)
+ public static TableInfo getTableInfo(AbsoluteTableIdentifier identifier)
throws IOException {
- CarbonTablePath carbonTablePath = CarbonStorePath.getCarbonTablePath(absoluteTableIdentifier);
org.apache.carbondata.format.TableInfo thriftTableInfo =
- CarbonUtil.readSchemaFile(carbonTablePath.getSchemaFilePath());
+ CarbonUtil.readSchemaFile(CarbonTablePath.getSchemaFilePath(identifier.getTablePath()));
ThriftWrapperSchemaConverterImpl thriftWrapperSchemaConverter =
new ThriftWrapperSchemaConverterImpl();
CarbonTableIdentifier carbonTableIdentifier =
- absoluteTableIdentifier.getCarbonTableIdentifier();
+ identifier.getCarbonTableIdentifier();
return thriftWrapperSchemaConverter.fromExternalToWrapperTableInfo(
thriftTableInfo,
carbonTableIdentifier.getDatabaseName(),
carbonTableIdentifier.getTableName(),
- absoluteTableIdentifier.getTablePath());
+ identifier.getTablePath());
}
}
http://git-wip-us.apache.org/repos/asf/carbondata/blob/2e1f2b95/hadoop/src/test/java/org/apache/carbondata/hadoop/test/util/StoreCreator.java
----------------------------------------------------------------------
diff --git a/hadoop/src/test/java/org/apache/carbondata/hadoop/test/util/StoreCreator.java b/hadoop/src/test/java/org/apache/carbondata/hadoop/test/util/StoreCreator.java
index ac17c4e..6365a6a 100644
--- a/hadoop/src/test/java/org/apache/carbondata/hadoop/test/util/StoreCreator.java
+++ b/hadoop/src/test/java/org/apache/carbondata/hadoop/test/util/StoreCreator.java
@@ -65,7 +65,6 @@ import org.apache.carbondata.core.statusmanager.LoadMetadataDetails;
import org.apache.carbondata.core.statusmanager.SegmentStatus;
import org.apache.carbondata.core.util.CarbonProperties;
import org.apache.carbondata.core.util.CarbonUtil;
-import org.apache.carbondata.core.util.path.CarbonStorePath;
import org.apache.carbondata.core.util.path.CarbonTablePath;
import org.apache.carbondata.core.writer.CarbonDictionaryWriter;
import org.apache.carbondata.core.writer.CarbonDictionaryWriterImpl;
@@ -188,11 +187,11 @@ public class StoreCreator {
}
public static CarbonTable createTable(
- AbsoluteTableIdentifier absoluteTableIdentifier) throws IOException {
+ AbsoluteTableIdentifier identifier) throws IOException {
TableInfo tableInfo = new TableInfo();
- tableInfo.setDatabaseName(absoluteTableIdentifier.getCarbonTableIdentifier().getDatabaseName());
+ tableInfo.setDatabaseName(identifier.getCarbonTableIdentifier().getDatabaseName());
TableSchema tableSchema = new TableSchema();
- tableSchema.setTableName(absoluteTableIdentifier.getCarbonTableIdentifier().getTableName());
+ tableSchema.setTableName(identifier.getCarbonTableIdentifier().getTableName());
List<ColumnSchema> columnSchemas = new ArrayList<ColumnSchema>();
ArrayList<Encoding> encodings = new ArrayList<>();
encodings.add(Encoding.DICTIONARY);
@@ -284,13 +283,12 @@ public class StoreCreator {
tableSchema.setSchemaEvalution(schemaEvol);
tableSchema.setTableId(UUID.randomUUID().toString());
tableInfo.setTableUniqueName(
- absoluteTableIdentifier.getCarbonTableIdentifier().getTableUniqueName()
+ identifier.getCarbonTableIdentifier().getTableUniqueName()
);
tableInfo.setLastUpdatedTime(System.currentTimeMillis());
tableInfo.setFactTable(tableSchema);
- tableInfo.setTablePath(absoluteTableIdentifier.getTablePath());
- CarbonTablePath carbonTablePath = CarbonStorePath.getCarbonTablePath(absoluteTableIdentifier);
- String schemaFilePath = carbonTablePath.getSchemaFilePath();
+ tableInfo.setTablePath(identifier.getTablePath());
+ String schemaFilePath = CarbonTablePath.getSchemaFilePath(identifier.getTablePath());
String schemaMetadataPath = CarbonTablePath.getFolderContainingFile(schemaFilePath);
CarbonMetadata.getInstance().loadTableMetadata(tableInfo);
@@ -469,7 +467,7 @@ public class StoreCreator {
loadMetadataDetails.setLoadStartTime(loadMetadataDetails.getTimeStamp(readCurrentTime()));
listOfLoadFolderDetails.add(loadMetadataDetails);
- String dataLoadLocation = schema.getCarbonTable().getMetaDataFilepath() + File.separator
+ String dataLoadLocation = schema.getCarbonTable().getMetadataPath() + File.separator
+ CarbonCommonConstants.LOADMETADATA_FILENAME;
DataOutputStream dataOutputStream;
http://git-wip-us.apache.org/repos/asf/carbondata/blob/2e1f2b95/integration/presto/src/main/java/org/apache/carbondata/presto/CarbondataRecordSetProvider.java
----------------------------------------------------------------------
diff --git a/integration/presto/src/main/java/org/apache/carbondata/presto/CarbondataRecordSetProvider.java b/integration/presto/src/main/java/org/apache/carbondata/presto/CarbondataRecordSetProvider.java
index 5a2f831..8f7e88c 100755
--- a/integration/presto/src/main/java/org/apache/carbondata/presto/CarbondataRecordSetProvider.java
+++ b/integration/presto/src/main/java/org/apache/carbondata/presto/CarbondataRecordSetProvider.java
@@ -25,7 +25,6 @@ import org.apache.carbondata.core.metadata.AbsoluteTableIdentifier;
import org.apache.carbondata.core.metadata.schema.table.CarbonTable;
import org.apache.carbondata.core.scan.expression.Expression;
import org.apache.carbondata.core.scan.model.QueryModel;
-import org.apache.carbondata.core.service.impl.PathFactory;
import org.apache.carbondata.hadoop.CarbonInputSplit;
import org.apache.carbondata.hadoop.CarbonProjection;
import org.apache.carbondata.hadoop.api.CarbonTableInputFormat;
@@ -82,7 +81,8 @@ public class CarbondataRecordSetProvider implements ConnectorRecordSetProvider {
carbonTableReader.getCarbonCache(carbondataSplit.getSchemaTableName());
checkNotNull(tableCacheModel, "tableCacheModel should not be null");
checkNotNull(tableCacheModel.carbonTable, "tableCacheModel.carbonTable should not be null");
- checkNotNull(tableCacheModel.tableInfo, "tableCacheModel.tableInfo should not be null");
+ checkNotNull(
+ tableCacheModel.carbonTable.getTableInfo(), "tableCacheModel.tableInfo should not be null");
// Build Query Model
CarbonTable targetTable = tableCacheModel.carbonTable;
@@ -92,8 +92,7 @@ public class CarbondataRecordSetProvider implements ConnectorRecordSetProvider {
try {
Configuration conf = new Configuration();
conf.set(CarbonTableInputFormat.INPUT_SEGMENT_NUMBERS, "");
- String carbonTablePath = PathFactory.getInstance()
- .getCarbonTablePath(targetTable.getAbsoluteTableIdentifier(), null).getPath();
+ String carbonTablePath = targetTable.getAbsoluteTableIdentifier().getTablePath();
conf.set(CarbonTableInputFormat.INPUT_DIR, carbonTablePath);
JobConf jobConf = new JobConf(conf);
http://git-wip-us.apache.org/repos/asf/carbondata/blob/2e1f2b95/integration/presto/src/main/java/org/apache/carbondata/presto/impl/CarbonTableCacheModel.java
----------------------------------------------------------------------
diff --git a/integration/presto/src/main/java/org/apache/carbondata/presto/impl/CarbonTableCacheModel.java b/integration/presto/src/main/java/org/apache/carbondata/presto/impl/CarbonTableCacheModel.java
index 8422c3e..4984406 100755
--- a/integration/presto/src/main/java/org/apache/carbondata/presto/impl/CarbonTableCacheModel.java
+++ b/integration/presto/src/main/java/org/apache/carbondata/presto/impl/CarbonTableCacheModel.java
@@ -17,27 +17,18 @@
package org.apache.carbondata.presto.impl;
-import org.apache.carbondata.core.metadata.CarbonTableIdentifier;
import org.apache.carbondata.core.metadata.schema.table.CarbonTable;
-import org.apache.carbondata.core.metadata.schema.table.TableInfo;
-import org.apache.carbondata.core.util.path.CarbonTablePath;
/**
- * Caching metadata of CarbonData(e.g. TableIdentifier, TablePath, TableInfo, CarbonTable) in Class CarbonTableReader
+ * Caching metadata of CarbonData in Class CarbonTableReader
* to speed up query
*/
public class CarbonTableCacheModel {
- public CarbonTableIdentifier carbonTableIdentifier;
- public CarbonTablePath carbonTablePath;
-
- public TableInfo tableInfo;
public CarbonTable carbonTable;
public boolean isValid() {
- if (carbonTable != null && carbonTablePath != null && carbonTableIdentifier != null)
- return true;
- else return false;
+ return carbonTable != null;
}
}
http://git-wip-us.apache.org/repos/asf/carbondata/blob/2e1f2b95/integration/presto/src/main/java/org/apache/carbondata/presto/impl/CarbonTableReader.java
----------------------------------------------------------------------
diff --git a/integration/presto/src/main/java/org/apache/carbondata/presto/impl/CarbonTableReader.java b/integration/presto/src/main/java/org/apache/carbondata/presto/impl/CarbonTableReader.java
index b0271ef..09389f8 100755
--- a/integration/presto/src/main/java/org/apache/carbondata/presto/impl/CarbonTableReader.java
+++ b/integration/presto/src/main/java/org/apache/carbondata/presto/impl/CarbonTableReader.java
@@ -42,7 +42,6 @@ import org.apache.carbondata.core.metadata.schema.table.CarbonTable;
import org.apache.carbondata.core.metadata.schema.table.TableInfo;
import org.apache.carbondata.core.reader.ThriftReader;
import org.apache.carbondata.core.scan.expression.Expression;
-import org.apache.carbondata.core.service.impl.PathFactory;
import org.apache.carbondata.core.util.path.CarbonTablePath;
import org.apache.carbondata.hadoop.CarbonInputSplit;
import org.apache.carbondata.hadoop.api.CarbonTableInputFormat;
@@ -243,9 +242,10 @@ public class CarbonTableReader {
updateSchemaList();
}
try {
- if (isKeyExists && !FileFactory
- .isFileExist(carbonCache.get().get(schemaTableName).carbonTablePath.getSchemaFilePath(),
- fileType)) {
+ if (isKeyExists
+ && !FileFactory.isFileExist(
+ CarbonTablePath.getSchemaFilePath(
+ carbonCache.get().get(schemaTableName).carbonTable.getTablePath()), fileType)) {
removeTableFromCache(schemaTableName);
throw new TableNotFoundException(schemaTableName);
}
@@ -255,10 +255,12 @@ public class CarbonTableReader {
if (isKeyExists) {
CarbonTableCacheModel ctcm = carbonCache.get().get(schemaTableName);
- if(ctcm != null && ctcm.tableInfo != null) {
- Long latestTime = FileFactory.getCarbonFile(ctcm.carbonTablePath.getSchemaFilePath())
- .getLastModifiedTime();
- Long oldTime = ctcm.tableInfo.getLastUpdatedTime();
+ if(ctcm != null && ctcm.carbonTable.getTableInfo() != null) {
+ Long latestTime = FileFactory.getCarbonFile(
+ CarbonTablePath.getSchemaFilePath(
+ carbonCache.get().get(schemaTableName).carbonTable.getTablePath())
+ ).getLastModifiedTime();
+ Long oldTime = ctcm.carbonTable.getTableInfo().getLastUpdatedTime();
if (DateUtils.truncate(new Date(latestTime), Calendar.MINUTE)
.after(DateUtils.truncate(new Date(oldTime), Calendar.MINUTE))) {
removeTableFromCache(schemaTableName);
@@ -312,21 +314,12 @@ public class CarbonTableReader {
// Step 1: get store path of the table and cache it.
// create table identifier. the table id is randomly generated.
- cache.carbonTableIdentifier =
+ CarbonTableIdentifier carbonTableIdentifier =
new CarbonTableIdentifier(table.getSchemaName(), table.getTableName(),
UUID.randomUUID().toString());
String storePath = config.getStorePath();
- String tablePath = storePath + "/" + cache.carbonTableIdentifier.getDatabaseName() + "/"
- + cache.carbonTableIdentifier.getTableName();
-
- // get the store path of the table.
-
- AbsoluteTableIdentifier absoluteTableIdentifier =
- AbsoluteTableIdentifier.from(tablePath, cache.carbonTableIdentifier);
- cache.carbonTablePath =
- PathFactory.getInstance().getCarbonTablePath(absoluteTableIdentifier, null);
- // cache the table
- carbonCache.get().put(table, cache);
+ String tablePath = storePath + "/" + carbonTableIdentifier.getDatabaseName() + "/"
+ + carbonTableIdentifier.getTableName();
//Step 2: read the metadata (tableInfo) of the table.
ThriftReader.TBaseCreator createTBase = new ThriftReader.TBaseCreator() {
@@ -338,7 +331,7 @@ public class CarbonTableReader {
}
};
ThriftReader thriftReader =
- new ThriftReader(cache.carbonTablePath.getSchemaFilePath(), createTBase);
+ new ThriftReader(CarbonTablePath.getSchemaFilePath(tablePath), createTBase);
thriftReader.open();
org.apache.carbondata.format.TableInfo tableInfo =
(org.apache.carbondata.format.TableInfo) thriftReader.read();
@@ -355,9 +348,12 @@ public class CarbonTableReader {
// Step 4: Load metadata info into CarbonMetadata
CarbonMetadata.getInstance().loadTableMetadata(wrapperTableInfo);
- cache.tableInfo = wrapperTableInfo;
- cache.carbonTable = CarbonMetadata.getInstance()
- .getCarbonTable(cache.carbonTableIdentifier.getTableUniqueName());
+ cache.carbonTable = CarbonMetadata.getInstance().getCarbonTable(
+ table.getSchemaName(), table.getTableName());
+
+ // cache the table
+ carbonCache.get().put(table, cache);
+
result = cache.carbonTable;
} catch (Exception ex) {
throw new RuntimeException(ex);
@@ -372,11 +368,10 @@ public class CarbonTableReader {
List<CarbonLocalInputSplit> result = new ArrayList<>();
CarbonTable carbonTable = tableCacheModel.carbonTable;
- TableInfo tableInfo = tableCacheModel.tableInfo;
+ TableInfo tableInfo = tableCacheModel.carbonTable.getTableInfo();
Configuration config = new Configuration();
config.set(CarbonTableInputFormat.INPUT_SEGMENT_NUMBERS, "");
- String carbonTablePath = PathFactory.getInstance()
- .getCarbonTablePath(carbonTable.getAbsoluteTableIdentifier(), null).getPath();
+ String carbonTablePath = carbonTable.getAbsoluteTableIdentifier().getTablePath();
config.set(CarbonTableInputFormat.INPUT_DIR, carbonTablePath);
config.set(CarbonTableInputFormat.DATABASE_NAME, carbonTable.getDatabaseName());
config.set(CarbonTableInputFormat.TABLE_NAME, carbonTable.getTableName());
http://git-wip-us.apache.org/repos/asf/carbondata/blob/2e1f2b95/integration/presto/src/test/scala/org/apache/carbondata/presto/util/CarbonDataStoreCreator.scala
----------------------------------------------------------------------
diff --git a/integration/presto/src/test/scala/org/apache/carbondata/presto/util/CarbonDataStoreCreator.scala b/integration/presto/src/test/scala/org/apache/carbondata/presto/util/CarbonDataStoreCreator.scala
index a41e738..1d7c791 100644
--- a/integration/presto/src/test/scala/org/apache/carbondata/presto/util/CarbonDataStoreCreator.scala
+++ b/integration/presto/src/test/scala/org/apache/carbondata/presto/util/CarbonDataStoreCreator.scala
@@ -52,7 +52,7 @@ import org.apache.carbondata.core.metadata.schema.{SchemaEvolution, SchemaEvolut
import org.apache.carbondata.core.metadata.{AbsoluteTableIdentifier, CarbonMetadata,
CarbonTableIdentifier, ColumnIdentifier}
import org.apache.carbondata.core.statusmanager.{LoadMetadataDetails, SegmentStatus}
-import org.apache.carbondata.core.util.path.{CarbonStorePath, CarbonTablePath}
+import org.apache.carbondata.core.util.path.CarbonTablePath
import org.apache.carbondata.core.util.{CarbonProperties, CarbonUtil}
import org.apache.carbondata.core.writer.sortindex.{CarbonDictionarySortIndexWriter,
CarbonDictionarySortIndexWriterImpl, CarbonDictionarySortInfo, CarbonDictionarySortInfoPreparator}
@@ -323,10 +323,8 @@ object CarbonDataStoreCreator {
)
tableInfo.setLastUpdatedTime(System.currentTimeMillis())
tableInfo.setFactTable(tableSchema)
- val carbonTablePath: CarbonTablePath = CarbonStorePath.getCarbonTablePath(
- absoluteTableIdentifier.getTablePath,
- absoluteTableIdentifier.getCarbonTableIdentifier)
- val schemaFilePath: String = carbonTablePath.getSchemaFilePath
+ val schemaFilePath: String = CarbonTablePath.getSchemaFilePath(
+ absoluteTableIdentifier.getTablePath)
val schemaMetadataPath: String =
CarbonTablePath.getFolderContainingFile(schemaFilePath)
CarbonMetadata.getInstance.loadTableMetadata(tableInfo)
@@ -533,7 +531,7 @@ object CarbonDataStoreCreator {
loadMetadataDetails.setLoadStartTime(
loadMetadataDetails.getTimeStamp(readCurrentTime()))
listOfLoadFolderDetails.add(loadMetadataDetails)
- val dataLoadLocation: String = schema.getCarbonTable.getMetaDataFilepath + File.separator +
+ val dataLoadLocation: String = schema.getCarbonTable.getMetadataPath + File.separator +
CarbonCommonConstants.LOADMETADATA_FILENAME
val gsonObjectToWrite: Gson = new Gson()
val writeOperation: AtomicFileOperations = new AtomicFileOperationsImpl(
http://git-wip-us.apache.org/repos/asf/carbondata/blob/2e1f2b95/integration/spark-common-cluster-test/src/test/scala/org/apache/carbondata/cluster/sdv/generated/MergeIndexTestCase.scala
----------------------------------------------------------------------
diff --git a/integration/spark-common-cluster-test/src/test/scala/org/apache/carbondata/cluster/sdv/generated/MergeIndexTestCase.scala b/integration/spark-common-cluster-test/src/test/scala/org/apache/carbondata/cluster/sdv/generated/MergeIndexTestCase.scala
index 8e71257..ed6d741 100644
--- a/integration/spark-common-cluster-test/src/test/scala/org/apache/carbondata/cluster/sdv/generated/MergeIndexTestCase.scala
+++ b/integration/spark-common-cluster-test/src/test/scala/org/apache/carbondata/cluster/sdv/generated/MergeIndexTestCase.scala
@@ -18,18 +18,14 @@
package org.apache.carbondata.cluster.sdv.generated
-import org.apache.spark.sql.CarbonEnv
import org.apache.spark.sql.common.util._
import org.scalatest.BeforeAndAfterAll
-import org.apache.carbondata.core.constants.CarbonCommonConstants
import org.apache.carbondata.core.datastore.filesystem.{CarbonFile, CarbonFileFilter}
import org.apache.carbondata.core.datastore.impl.FileFactory
-import org.apache.carbondata.core.indexstore.blockletindex.SegmentIndexFileStore
-import org.apache.carbondata.core.metadata.{AbsoluteTableIdentifier, CarbonMetadata}
-import org.apache.carbondata.core.util.CarbonProperties
-import org.apache.carbondata.core.util.path.{CarbonStorePath, CarbonTablePath}
+import org.apache.carbondata.core.metadata.CarbonMetadata
import org.apache.carbondata.core.writer.CarbonIndexFileMergeWriter
+import org.apache.carbondata.core.util.path.CarbonTablePath
/**
* Test Class for AlterTableTestCase to verify all scenerios
http://git-wip-us.apache.org/repos/asf/carbondata/blob/2e1f2b95/integration/spark-common-test/src/test/scala/org/apache/carbondata/integration/spark/testsuite/dataload/TestLoadDataGeneral.scala
----------------------------------------------------------------------
diff --git a/integration/spark-common-test/src/test/scala/org/apache/carbondata/integration/spark/testsuite/dataload/TestLoadDataGeneral.scala b/integration/spark-common-test/src/test/scala/org/apache/carbondata/integration/spark/testsuite/dataload/TestLoadDataGeneral.scala
index c84ae6b..a707c60 100644
--- a/integration/spark-common-test/src/test/scala/org/apache/carbondata/integration/spark/testsuite/dataload/TestLoadDataGeneral.scala
+++ b/integration/spark-common-test/src/test/scala/org/apache/carbondata/integration/spark/testsuite/dataload/TestLoadDataGeneral.scala
@@ -24,7 +24,7 @@ import scala.collection.mutable.ArrayBuffer
import org.apache.spark.sql.Row
import org.scalatest.BeforeAndAfterEach
-import org.apache.carbondata.core.util.path.{CarbonStorePath, CarbonTablePath}
+import org.apache.carbondata.core.util.path.CarbonTablePath
import org.apache.carbondata.core.datastore.impl.FileFactory
import org.apache.carbondata.core.metadata.CarbonMetadata
import org.apache.spark.sql.test.util.QueryTest
@@ -48,13 +48,13 @@ class TestLoadDataGeneral extends QueryTest with BeforeAndAfterEach {
datbaseName: String,
tableName: String): Boolean = {
val carbonTable = CarbonMetadata.getInstance().getCarbonTable(datbaseName, tableName)
- val partitionPath = CarbonStorePath
- .getCarbonTablePath(carbonTable.getAbsoluteTableIdentifier).getPartitionDir
+ val partitionPath =
+ CarbonTablePath.getPartitionDir(carbonTable.getAbsoluteTableIdentifier.getTablePath)
val fileType: FileFactory.FileType = FileFactory.getFileType(partitionPath)
val carbonFile = FileFactory.getCarbonFile(partitionPath, fileType)
val segments: ArrayBuffer[String] = ArrayBuffer()
carbonFile.listFiles.foreach { file =>
- segments += CarbonTablePath.DataPathUtil.getSegmentId(file.getAbsolutePath + "/dummy")
+ segments += CarbonTablePath.DataFileUtil.getSegmentId(file.getAbsolutePath + "/dummy")
}
segments.contains(segmentId)
}
@@ -235,8 +235,7 @@ class TestLoadDataGeneral extends QueryTest with BeforeAndAfterEach {
sql("create table stale(a string) stored by 'carbondata'")
sql("insert into stale values('k')")
val carbonTable = CarbonMetadata.getInstance().getCarbonTable("default", "stale")
- val tableStatusFile = new CarbonTablePath(null,
- carbonTable.getTablePath).getTableStatusFilePath
+ val tableStatusFile = CarbonTablePath.getTableStatusFilePath(carbonTable.getTablePath)
FileFactory.getCarbonFile(tableStatusFile).delete()
sql("insert into stale values('k')")
checkAnswer(sql("select * from stale"), Row("k"))
http://git-wip-us.apache.org/repos/asf/carbondata/blob/2e1f2b95/integration/spark-common-test/src/test/scala/org/apache/carbondata/spark/testsuite/allqueries/InsertIntoCarbonTableTestCase.scala
----------------------------------------------------------------------
diff --git a/integration/spark-common-test/src/test/scala/org/apache/carbondata/spark/testsuite/allqueries/InsertIntoCarbonTableTestCase.scala b/integration/spark-common-test/src/test/scala/org/apache/carbondata/spark/testsuite/allqueries/InsertIntoCarbonTableTestCase.scala
index 5cc4156..3c2fd71 100644
--- a/integration/spark-common-test/src/test/scala/org/apache/carbondata/spark/testsuite/allqueries/InsertIntoCarbonTableTestCase.scala
+++ b/integration/spark-common-test/src/test/scala/org/apache/carbondata/spark/testsuite/allqueries/InsertIntoCarbonTableTestCase.scala
@@ -29,7 +29,7 @@ import org.apache.carbondata.core.datastore.filesystem.CarbonFile
import org.apache.carbondata.core.datastore.impl.FileFactory
import org.apache.carbondata.core.metadata.CarbonMetadata
import org.apache.carbondata.core.metadata.schema.table.CarbonTable
-import org.apache.carbondata.core.util.path.CarbonStorePath
+import org.apache.carbondata.core.util.path.CarbonTablePath
class InsertIntoCarbonTableTestCase extends QueryTest with BeforeAndAfterAll {
var timeStampPropOrig: String = _
@@ -231,8 +231,7 @@ class InsertIntoCarbonTableTestCase extends QueryTest with BeforeAndAfterAll {
sql("insert overwrite table HiveOverwrite select * from THive")
checkAnswer(sql("select count(*) from CarbonOverwrite"), sql("select count(*) from HiveOverwrite"))
val carbonTable = CarbonMetadata.getInstance().getCarbonTable("default", "carbonoverwrite")
- val partitionPath = CarbonStorePath
- .getCarbonTablePath(carbonTable.getAbsoluteTableIdentifier).getPartitionDir
+ val partitionPath = CarbonTablePath.getPartitionDir(carbonTable.getAbsoluteTableIdentifier.getTablePath)
val folder = new File(partitionPath)
assert(folder.isDirectory)
assert(folder.list().length == 1)
@@ -254,8 +253,7 @@ class InsertIntoCarbonTableTestCase extends QueryTest with BeforeAndAfterAll {
sql(s"LOAD DATA local INPATH '$resourcesPath/100_olap.csv' overwrite INTO TABLE HiveOverwrite")
checkAnswer(sql("select count(*) from TCarbonSourceOverwrite"), sql("select count(*) from HiveOverwrite"))
val carbonTable = CarbonMetadata.getInstance().getCarbonTable("default", "tcarbonsourceoverwrite")
- val partitionPath = CarbonStorePath
- .getCarbonTablePath(carbonTable.getAbsoluteTableIdentifier).getPartitionDir
+ val partitionPath = CarbonTablePath.getPartitionDir(carbonTable.getAbsoluteTableIdentifier.getTablePath)
val folder = new File(partitionPath)
assert(folder.isDirectory)
http://git-wip-us.apache.org/repos/asf/carbondata/blob/2e1f2b95/integration/spark-common-test/src/test/scala/org/apache/carbondata/spark/testsuite/createTable/TestCreateTableAsSelect.scala
----------------------------------------------------------------------
diff --git a/integration/spark-common-test/src/test/scala/org/apache/carbondata/spark/testsuite/createTable/TestCreateTableAsSelect.scala b/integration/spark-common-test/src/test/scala/org/apache/carbondata/spark/testsuite/createTable/TestCreateTableAsSelect.scala
index 09268b5..8315848 100644
--- a/integration/spark-common-test/src/test/scala/org/apache/carbondata/spark/testsuite/createTable/TestCreateTableAsSelect.scala
+++ b/integration/spark-common-test/src/test/scala/org/apache/carbondata/spark/testsuite/createTable/TestCreateTableAsSelect.scala
@@ -142,7 +142,7 @@ class TestCreateTableAsSelect extends QueryTest with BeforeAndAfterAll {
val carbonTable = CarbonEnv.getInstance(Spark2TestQueryExecutor.spark).carbonMetastore
.lookupRelation(Option("default"), "ctas_tblproperties_test")(Spark2TestQueryExecutor.spark)
.asInstanceOf[CarbonRelation].carbonTable
- val metadataFolderPath: CarbonFile = FileFactory.getCarbonFile(carbonTable.getMetaDataFilepath)
+ val metadataFolderPath: CarbonFile = FileFactory.getCarbonFile(carbonTable.getMetadataPath)
assert(metadataFolderPath.exists())
val dictFiles: Array[CarbonFile] = metadataFolderPath.listFiles(new CarbonFileFilter {
override def accept(file: CarbonFile): Boolean = {
http://git-wip-us.apache.org/repos/asf/carbondata/blob/2e1f2b95/integration/spark-common-test/src/test/scala/org/apache/carbondata/spark/testsuite/datacompaction/DataCompactionLockTest.scala
----------------------------------------------------------------------
diff --git a/integration/spark-common-test/src/test/scala/org/apache/carbondata/spark/testsuite/datacompaction/DataCompactionLockTest.scala b/integration/spark-common-test/src/test/scala/org/apache/carbondata/spark/testsuite/datacompaction/DataCompactionLockTest.scala
index 2ef88a4..a7607c3 100644
--- a/integration/spark-common-test/src/test/scala/org/apache/carbondata/spark/testsuite/datacompaction/DataCompactionLockTest.scala
+++ b/integration/spark-common-test/src/test/scala/org/apache/carbondata/spark/testsuite/datacompaction/DataCompactionLockTest.scala
@@ -24,7 +24,7 @@ import org.scalatest.BeforeAndAfterAll
import org.apache.spark.sql.test.util.QueryTest
-import org.apache.carbondata.core.util.path.{CarbonStorePath, CarbonTablePath}
+import org.apache.carbondata.core.util.path.CarbonTablePath
import org.apache.carbondata.core.metadata.{AbsoluteTableIdentifier, CarbonTableIdentifier}
import org.apache.carbondata.core.constants.CarbonCommonConstants
import org.apache.carbondata.core.statusmanager.SegmentStatusManager
@@ -43,9 +43,7 @@ class DataCompactionLockTest extends QueryTest with BeforeAndAfterAll {
new CarbonTableIdentifier(
CarbonCommonConstants.DATABASE_DEFAULT_NAME, "compactionlocktesttable", "1")
)
- val carbonTablePath: CarbonTablePath = CarbonStorePath
- .getCarbonTablePath(absoluteTableIdentifier)
- val dataPath: String = carbonTablePath.getMetadataDirectoryPath
+ val dataPath: String = CarbonTablePath.getMetadataPath(absoluteTableIdentifier.getTablePath)
val carbonLock: ICarbonLock =
CarbonLockFactory
http://git-wip-us.apache.org/repos/asf/carbondata/blob/2e1f2b95/integration/spark-common-test/src/test/scala/org/apache/carbondata/spark/testsuite/datacompaction/MajorCompactionIgnoreInMinorTest.scala
----------------------------------------------------------------------
diff --git a/integration/spark-common-test/src/test/scala/org/apache/carbondata/spark/testsuite/datacompaction/MajorCompactionIgnoreInMinorTest.scala b/integration/spark-common-test/src/test/scala/org/apache/carbondata/spark/testsuite/datacompaction/MajorCompactionIgnoreInMinorTest.scala
index 31a08fc..9afb890 100644
--- a/integration/spark-common-test/src/test/scala/org/apache/carbondata/spark/testsuite/datacompaction/MajorCompactionIgnoreInMinorTest.scala
+++ b/integration/spark-common-test/src/test/scala/org/apache/carbondata/spark/testsuite/datacompaction/MajorCompactionIgnoreInMinorTest.scala
@@ -26,7 +26,7 @@ import org.apache.carbondata.core.datastore.TableSegmentUniqueIdentifier
import org.apache.carbondata.core.metadata.CarbonMetadata
import org.apache.carbondata.core.statusmanager.{SegmentStatus, SegmentStatusManager}
import org.apache.carbondata.core.util.CarbonProperties
-import org.apache.carbondata.core.util.path.CarbonStorePath
+import org.apache.carbondata.core.util.path.CarbonTablePath
import org.apache.carbondata.hadoop.CacheClient
import org.apache.spark.sql.test.util.QueryTest
@@ -112,11 +112,9 @@ class MajorCompactionIgnoreInMinorTest extends QueryTest with BeforeAndAfterAll
CarbonCommonConstants.DATABASE_DEFAULT_NAME,
"ignoremajor"
)
- val absoluteTableIdentifier = carbonTable.getAbsoluteTableIdentifier
- val carbontablePath = CarbonStorePath.getCarbonTablePath(absoluteTableIdentifier)
- .getMetadataDirectoryPath
- val segs = SegmentStatusManager.readLoadMetadata(carbontablePath)
+ val carbonTablePath = carbonTable.getMetadataPath
+ val segs = SegmentStatusManager.readLoadMetadata(carbonTablePath)
// status should remain as compacted.
assertResult(SegmentStatus.COMPACTED)(segs(3).getSegmentStatus)
@@ -134,9 +132,7 @@ class MajorCompactionIgnoreInMinorTest extends QueryTest with BeforeAndAfterAll
CarbonCommonConstants.DATABASE_DEFAULT_NAME,
"ignoremajor"
)
- val absoluteTableIdentifier = carbonTable.getAbsoluteTableIdentifier
- val carbontablePath = CarbonStorePath
- .getCarbonTablePath(absoluteTableIdentifier).getMetadataDirectoryPath
+ val carbontablePath = carbonTable.getMetadataPath
val segs = SegmentStatusManager.readLoadMetadata(carbontablePath)
// status should remain as compacted for segment 2.
http://git-wip-us.apache.org/repos/asf/carbondata/blob/2e1f2b95/integration/spark-common-test/src/test/scala/org/apache/carbondata/spark/testsuite/dataload/TestBatchSortDataLoad.scala
----------------------------------------------------------------------
diff --git a/integration/spark-common-test/src/test/scala/org/apache/carbondata/spark/testsuite/dataload/TestBatchSortDataLoad.scala b/integration/spark-common-test/src/test/scala/org/apache/carbondata/spark/testsuite/dataload/TestBatchSortDataLoad.scala
index 42ac4df..68a3058 100644
--- a/integration/spark-common-test/src/test/scala/org/apache/carbondata/spark/testsuite/dataload/TestBatchSortDataLoad.scala
+++ b/integration/spark-common-test/src/test/scala/org/apache/carbondata/spark/testsuite/dataload/TestBatchSortDataLoad.scala
@@ -29,7 +29,7 @@ import org.apache.spark.sql.test.util.QueryTest
import org.apache.carbondata.core.indexstore.blockletindex.SegmentIndexFileStore
import org.apache.carbondata.core.metadata.CarbonMetadata
import org.apache.carbondata.core.metadata.schema.table.CarbonTable
-import org.apache.carbondata.core.util.path.CarbonStorePath
+import org.apache.carbondata.core.util.path.CarbonTablePath
class TestBatchSortDataLoad extends QueryTest with BeforeAndAfterAll {
var filePath: String = _
@@ -193,8 +193,7 @@ class TestBatchSortDataLoad extends QueryTest with BeforeAndAfterAll {
CarbonCommonConstants.DATABASE_DEFAULT_NAME,
tableName
)
- val carbonTablePath = CarbonStorePath.getCarbonTablePath(carbonTable.getAbsoluteTableIdentifier)
- val segmentDir = carbonTablePath.getCarbonDataDirectoryPath(segmentNo)
+ val segmentDir = carbonTable.getSemgentPath(segmentNo)
new SegmentIndexFileStore().getIndexFilesFromSegment(segmentDir).size()
}
http://git-wip-us.apache.org/repos/asf/carbondata/blob/2e1f2b95/integration/spark-common-test/src/test/scala/org/apache/carbondata/spark/testsuite/dataload/TestDataLoadWithFileName.scala
----------------------------------------------------------------------
diff --git a/integration/spark-common-test/src/test/scala/org/apache/carbondata/spark/testsuite/dataload/TestDataLoadWithFileName.scala b/integration/spark-common-test/src/test/scala/org/apache/carbondata/spark/testsuite/dataload/TestDataLoadWithFileName.scala
index db0a62c..b9d8e12 100644
--- a/integration/spark-common-test/src/test/scala/org/apache/carbondata/spark/testsuite/dataload/TestDataLoadWithFileName.scala
+++ b/integration/spark-common-test/src/test/scala/org/apache/carbondata/spark/testsuite/dataload/TestDataLoadWithFileName.scala
@@ -22,7 +22,7 @@ import java.io.{File, FilenameFilter}
import org.apache.carbondata.core.constants.CarbonCommonConstants
import org.apache.carbondata.core.reader.CarbonIndexFileReader
import org.apache.carbondata.core.util.CarbonProperties
-import org.apache.carbondata.core.util.path.{CarbonStorePath, CarbonTablePath}
+import org.apache.carbondata.core.util.path.CarbonTablePath
import org.apache.spark.sql.test.util.QueryTest
import org.scalatest.BeforeAndAfterAll
@@ -48,8 +48,7 @@ class TestDataLoadWithFileName extends QueryTest with BeforeAndAfterAll {
sql(s"LOAD DATA LOCAL INPATH '$testData' into table test_table_v3")
val indexReader = new CarbonIndexFileReader()
val carbonTable = CarbonMetadata.getInstance().getCarbonTable("default", "test_table_v3")
- val carbonTablePath = CarbonStorePath.getCarbonTablePath(carbonTable.getAbsoluteTableIdentifier)
- val segmentDir = carbonTablePath.getCarbonDataDirectoryPath("0")
+ val segmentDir = CarbonTablePath.getSegmentPath(carbonTable.getTablePath, "0")
val carbonIndexPaths = new File(segmentDir)
.listFiles(new FilenameFilter {
override def accept(dir: File, name: String): Boolean = {
http://git-wip-us.apache.org/repos/asf/carbondata/blob/2e1f2b95/integration/spark-common-test/src/test/scala/org/apache/carbondata/spark/testsuite/dataload/TestGlobalSortDataLoad.scala
----------------------------------------------------------------------
diff --git a/integration/spark-common-test/src/test/scala/org/apache/carbondata/spark/testsuite/dataload/TestGlobalSortDataLoad.scala b/integration/spark-common-test/src/test/scala/org/apache/carbondata/spark/testsuite/dataload/TestGlobalSortDataLoad.scala
index 479db50..cbbb191 100644
--- a/integration/spark-common-test/src/test/scala/org/apache/carbondata/spark/testsuite/dataload/TestGlobalSortDataLoad.scala
+++ b/integration/spark-common-test/src/test/scala/org/apache/carbondata/spark/testsuite/dataload/TestGlobalSortDataLoad.scala
@@ -32,8 +32,8 @@ import org.scalatest.{BeforeAndAfterAll, BeforeAndAfterEach}
import org.apache.carbondata.core.indexstore.blockletindex.SegmentIndexFileStore
import org.apache.carbondata.core.metadata.CarbonMetadata
-import org.apache.carbondata.core.util.path.CarbonStorePath
import org.apache.carbondata.spark.rdd.CarbonScanRDD
+import org.apache.carbondata.core.util.path.CarbonTablePath
class TestGlobalSortDataLoad extends QueryTest with BeforeAndAfterEach with BeforeAndAfterAll {
var filePath: String = s"$resourcesPath/globalsort"
@@ -271,8 +271,7 @@ class TestGlobalSortDataLoad extends QueryTest with BeforeAndAfterEach with Befo
}
sql(s"LOAD DATA LOCAL INPATH '$inputPath' INTO TABLE carbon_globalsort")
val carbonTable = CarbonMetadata.getInstance().getCarbonTable("default", "carbon_globalsort")
- val carbonTablePath = CarbonStorePath.getCarbonTablePath(carbonTable.getAbsoluteTableIdentifier)
- val segmentDir = carbonTablePath.getSegmentDir("0")
+ val segmentDir = CarbonTablePath.getSegmentPath(carbonTable.getTablePath, "0")
assertResult(Math.max(7, defaultParallelism) + 1)(new File(segmentDir).listFiles().length)
}
@@ -378,8 +377,7 @@ class TestGlobalSortDataLoad extends QueryTest with BeforeAndAfterEach with Befo
private def getIndexFileCount(tableName: String, segmentNo: String = "0"): Int = {
val carbonTable = CarbonMetadata.getInstance().getCarbonTable("default", tableName)
- val carbonTablePath = CarbonStorePath.getCarbonTablePath(carbonTable.getAbsoluteTableIdentifier)
- val segmentDir = carbonTablePath.getCarbonDataDirectoryPath(segmentNo)
+ val segmentDir = CarbonTablePath.getSegmentPath(carbonTable.getTablePath, segmentNo)
new SegmentIndexFileStore().getIndexFilesFromSegment(segmentDir).size()
}
}
http://git-wip-us.apache.org/repos/asf/carbondata/blob/2e1f2b95/integration/spark-common-test/src/test/scala/org/apache/carbondata/spark/testsuite/dataretention/DataRetentionTestCase.scala
----------------------------------------------------------------------
diff --git a/integration/spark-common-test/src/test/scala/org/apache/carbondata/spark/testsuite/dataretention/DataRetentionTestCase.scala b/integration/spark-common-test/src/test/scala/org/apache/carbondata/spark/testsuite/dataretention/DataRetentionTestCase.scala
index ed58253..7c82f75 100644
--- a/integration/spark-common-test/src/test/scala/org/apache/carbondata/spark/testsuite/dataretention/DataRetentionTestCase.scala
+++ b/integration/spark-common-test/src/test/scala/org/apache/carbondata/spark/testsuite/dataretention/DataRetentionTestCase.scala
@@ -25,7 +25,7 @@ import org.apache.commons.lang3.time.DateUtils
import org.apache.spark.sql.Row
import org.scalatest.BeforeAndAfterAll
-import org.apache.carbondata.core.util.path.{CarbonStorePath, CarbonTablePath}
+import org.apache.carbondata.core.util.path.CarbonTablePath
import org.apache.carbondata.core.metadata.{AbsoluteTableIdentifier, CarbonMetadata, CarbonTableIdentifier}
import org.apache.carbondata.core.constants.CarbonCommonConstants
import org.apache.carbondata.core.util.CarbonProperties
@@ -79,8 +79,8 @@ class DataRetentionTestCase extends QueryTest with BeforeAndAfterAll {
"dataRetentionTable"
)
absoluteTableIdentifierForRetention = carbonTable2.getAbsoluteTableIdentifier
- carbonTablePath = CarbonStorePath
- .getCarbonTablePath(absoluteTableIdentifierForRetention).getMetadataDirectoryPath
+ carbonTablePath = CarbonTablePath
+ .getMetadataPath(absoluteTableIdentifierForRetention.getTablePath)
carbonTableStatusLock = CarbonLockFactory
.getCarbonLockObj(absoluteTableIdentifierForLock, LockUsage.TABLE_STATUS_LOCK)
carbonDeleteSegmentLock= CarbonLockFactory
http://git-wip-us.apache.org/repos/asf/carbondata/blob/2e1f2b95/integration/spark-common-test/src/test/scala/org/apache/carbondata/spark/testsuite/partition/TestDataLoadingForPartitionTable.scala
----------------------------------------------------------------------
diff --git a/integration/spark-common-test/src/test/scala/org/apache/carbondata/spark/testsuite/partition/TestDataLoadingForPartitionTable.scala b/integration/spark-common-test/src/test/scala/org/apache/carbondata/spark/testsuite/partition/TestDataLoadingForPartitionTable.scala
index 0a21aed..e5de8da 100644
--- a/integration/spark-common-test/src/test/scala/org/apache/carbondata/spark/testsuite/partition/TestDataLoadingForPartitionTable.scala
+++ b/integration/spark-common-test/src/test/scala/org/apache/carbondata/spark/testsuite/partition/TestDataLoadingForPartitionTable.scala
@@ -61,9 +61,7 @@ class TestDataLoadingForPartitionTable extends QueryTest with BeforeAndAfterAll
def validateDataFiles(tableUniqueName: String, segmentId: String, partitions: Seq[Int]): Unit = {
val carbonTable = CarbonMetadata.getInstance().getCarbonTable(tableUniqueName)
- val tablePath = new CarbonTablePath(carbonTable.getCarbonTableIdentifier,
- carbonTable.getTablePath)
- val segmentDir = tablePath.getCarbonDataDirectoryPath(segmentId)
+ val segmentDir = carbonTable.getSemgentPath(segmentId)
val carbonFile = FileFactory.getCarbonFile(segmentDir, FileFactory.getFileType(segmentDir))
val dataFiles = carbonFile.listFiles(new CarbonFileFilter() {
override def accept(file: CarbonFile): Boolean = {
http://git-wip-us.apache.org/repos/asf/carbondata/blob/2e1f2b95/integration/spark-common-test/src/test/scala/org/apache/carbondata/spark/testsuite/standardpartition/StandardPartitionTableLoadingTestCase.scala
----------------------------------------------------------------------
diff --git a/integration/spark-common-test/src/test/scala/org/apache/carbondata/spark/testsuite/standardpartition/StandardPartitionTableLoadingTestCase.scala b/integration/spark-common-test/src/test/scala/org/apache/carbondata/spark/testsuite/standardpartition/StandardPartitionTableLoadingTestCase.scala
index c8f7be3..2ce46ef 100644
--- a/integration/spark-common-test/src/test/scala/org/apache/carbondata/spark/testsuite/standardpartition/StandardPartitionTableLoadingTestCase.scala
+++ b/integration/spark-common-test/src/test/scala/org/apache/carbondata/spark/testsuite/standardpartition/StandardPartitionTableLoadingTestCase.scala
@@ -72,8 +72,6 @@ class StandardPartitionTableLoadingTestCase extends QueryTest with BeforeAndAfte
def validateDataFiles(tableUniqueName: String, segmentId: String, partition: Int): Unit = {
val carbonTable = CarbonMetadata.getInstance().getCarbonTable(tableUniqueName)
- val tablePath = new CarbonTablePath(carbonTable.getCarbonTableIdentifier,
- carbonTable.getTablePath)
val partitions = CarbonFilters
.getPartitions(Seq.empty,
sqlContext.sparkSession,
@@ -334,9 +332,7 @@ class StandardPartitionTableLoadingTestCase extends QueryTest with BeforeAndAfte
sql(s"""LOAD DATA local inpath '$resourcesPath/data.csv' INTO TABLE mergeindexpartitionthree OPTIONS('DELIMITER'= ',', 'QUOTECHAR'= '"')""")
val carbonTable = CarbonMetadata.getInstance().getCarbonTable("default_mergeindexpartitionthree")
- val tablePath = new CarbonTablePath(carbonTable.getCarbonTableIdentifier,
- carbonTable.getTablePath)
- val details = SegmentStatusManager.readTableStatusFile(tablePath.getTableStatusFilePath)
+ val details = SegmentStatusManager.readTableStatusFile(CarbonTablePath.getTableStatusFilePath(carbonTable.getTablePath))
val store = new SegmentFileStore(carbonTable.getTablePath, details(0).getSegmentFile)
store.readIndexFiles()
store.getIndexFiles
http://git-wip-us.apache.org/repos/asf/carbondata/blob/2e1f2b95/integration/spark-common/src/main/scala/org/apache/carbondata/api/CarbonStore.scala
----------------------------------------------------------------------
diff --git a/integration/spark-common/src/main/scala/org/apache/carbondata/api/CarbonStore.scala b/integration/spark-common/src/main/scala/org/apache/carbondata/api/CarbonStore.scala
index 5fc7e3d..8adcb11 100644
--- a/integration/spark-common/src/main/scala/org/apache/carbondata/api/CarbonStore.scala
+++ b/integration/spark-common/src/main/scala/org/apache/carbondata/api/CarbonStore.scala
@@ -177,7 +177,7 @@ object CarbonStore {
LOGGER.audit(s"Delete segment by Id request has been received for $dbName.$tableName")
validateLoadIds(loadids)
- val path = carbonTable.getMetaDataFilepath
+ val path = carbonTable.getMetadataPath
try {
val invalidLoadIds = SegmentStatusManager.updateDeletionStatus(
@@ -203,7 +203,7 @@ object CarbonStore {
LOGGER.audit(s"Delete segment by Id request has been received for $dbName.$tableName")
val time = validateTimeFormat(timestamp)
- val path = carbonTable.getMetaDataFilepath
+ val path = carbonTable.getMetadataPath
try {
val invalidLoadTimestamps =
http://git-wip-us.apache.org/repos/asf/carbondata/blob/2e1f2b95/integration/spark-common/src/main/scala/org/apache/carbondata/spark/CarbonSparkFactory.scala
----------------------------------------------------------------------
diff --git a/integration/spark-common/src/main/scala/org/apache/carbondata/spark/CarbonSparkFactory.scala b/integration/spark-common/src/main/scala/org/apache/carbondata/spark/CarbonSparkFactory.scala
index 32d121e..3dd9903 100644
--- a/integration/spark-common/src/main/scala/org/apache/carbondata/spark/CarbonSparkFactory.scala
+++ b/integration/spark-common/src/main/scala/org/apache/carbondata/spark/CarbonSparkFactory.scala
@@ -30,7 +30,7 @@ trait ColumnValidator {
*/
trait DictionaryDetailService {
def getDictionaryDetail(dictFolderPath: String, primDimensions: Array[CarbonDimension],
- table: CarbonTableIdentifier, storePath: String): DictionaryDetail
+ tablePath: String): DictionaryDetail
}
/**
http://git-wip-us.apache.org/repos/asf/carbondata/blob/2e1f2b95/integration/spark-common/src/main/scala/org/apache/carbondata/spark/DictionaryDetailHelper.scala
----------------------------------------------------------------------
diff --git a/integration/spark-common/src/main/scala/org/apache/carbondata/spark/DictionaryDetailHelper.scala b/integration/spark-common/src/main/scala/org/apache/carbondata/spark/DictionaryDetailHelper.scala
index e861a8c..dbf47ab 100644
--- a/integration/spark-common/src/main/scala/org/apache/carbondata/spark/DictionaryDetailHelper.scala
+++ b/integration/spark-common/src/main/scala/org/apache/carbondata/spark/DictionaryDetailHelper.scala
@@ -23,12 +23,11 @@ import org.apache.carbondata.core.datastore.filesystem.{CarbonFile, CarbonFileFi
import org.apache.carbondata.core.datastore.impl.FileFactory
import org.apache.carbondata.core.metadata.{CarbonTableIdentifier, ColumnIdentifier}
import org.apache.carbondata.core.metadata.schema.table.column.CarbonDimension
-import org.apache.carbondata.core.util.path.{CarbonStorePath, CarbonTablePath}
+import org.apache.carbondata.core.util.path.CarbonTablePath
class DictionaryDetailHelper extends DictionaryDetailService {
- def getDictionaryDetail(dictfolderPath: String, primDimensions: Array[CarbonDimension],
- table: CarbonTableIdentifier, storePath: String): DictionaryDetail = {
- val carbonTablePath = CarbonStorePath.getCarbonTablePath(storePath, table)
+ override def getDictionaryDetail(dictfolderPath: String, primDimensions: Array[CarbonDimension],
+ tablePath: String): DictionaryDetail = {
val dictFilePaths = new Array[String](primDimensions.length)
val dictFileExists = new Array[Boolean](primDimensions.length)
val columnIdentifier = new Array[ColumnIdentifier](primDimensions.length)
@@ -50,7 +49,7 @@ class DictionaryDetailHelper extends DictionaryDetailService {
// 3 lookup fileNamesMap, if file name is in fileNamesMap, file is exists, or not.
primDimensions.zipWithIndex.foreach { f =>
columnIdentifier(f._2) = f._1.getColumnIdentifier
- dictFilePaths(f._2) = carbonTablePath.getDictionaryFilePath(f._1.getColumnId)
+ dictFilePaths(f._2) = CarbonTablePath.getDictionaryFilePath(tablePath, f._1.getColumnId)
dictFileExists(f._2) =
fileNamesMap.get(CarbonTablePath.getDictionaryFileName(f._1.getColumnId)) match {
case None => false
http://git-wip-us.apache.org/repos/asf/carbondata/blob/2e1f2b95/integration/spark-common/src/main/scala/org/apache/carbondata/spark/rdd/AlterTableAddColumnRDD.scala
----------------------------------------------------------------------
diff --git a/integration/spark-common/src/main/scala/org/apache/carbondata/spark/rdd/AlterTableAddColumnRDD.scala b/integration/spark-common/src/main/scala/org/apache/carbondata/spark/rdd/AlterTableAddColumnRDD.scala
index f2f4ecd..56a66b9 100644
--- a/integration/spark-common/src/main/scala/org/apache/carbondata/spark/rdd/AlterTableAddColumnRDD.scala
+++ b/integration/spark-common/src/main/scala/org/apache/carbondata/spark/rdd/AlterTableAddColumnRDD.scala
@@ -26,7 +26,7 @@ import org.apache.carbondata.core.metadata.encoder.Encoding
import org.apache.carbondata.core.metadata.schema.table.column.ColumnSchema
import org.apache.carbondata.core.statusmanager.SegmentStatus
import org.apache.carbondata.core.util.CarbonProperties
-import org.apache.carbondata.core.util.path.CarbonStorePath
+import org.apache.carbondata.core.util.path.CarbonTablePath
import org.apache.carbondata.spark.util.GlobalDictionaryUtil
/**
@@ -49,7 +49,7 @@ class AddColumnPartition(rddId: Int, idx: Int, schema: ColumnSchema) extends Par
*/
class AlterTableAddColumnRDD[K, V](sc: SparkContext,
@transient newColumns: Seq[ColumnSchema],
- absoluteTableIdentifier: AbsoluteTableIdentifier)
+ identifier: AbsoluteTableIdentifier)
extends CarbonRDD[(Int, SegmentStatus)](sc, Nil) {
val lockType: String = CarbonProperties.getInstance.getProperty(CarbonCommonConstants.LOCK_TYPE,
@@ -70,8 +70,6 @@ class AlterTableAddColumnRDD[K, V](sc: SparkContext,
// create dictionary file if it is a dictionary column
if (columnSchema.hasEncoding(Encoding.DICTIONARY) &&
!columnSchema.hasEncoding(Encoding.DIRECT_DICTIONARY)) {
- val carbonTablePath = CarbonStorePath
- .getCarbonTablePath(absoluteTableIdentifier)
var rawData: String = null
if (null != columnSchema.getDefaultValue) {
rawData = new String(columnSchema.getDefaultValue,
@@ -79,16 +77,15 @@ class AlterTableAddColumnRDD[K, V](sc: SparkContext,
}
CarbonProperties.getInstance.addProperty(CarbonCommonConstants.LOCK_TYPE, lockType)
// Create table and metadata folders if not exist
- val metadataDirectoryPath = carbonTablePath.getMetadataDirectoryPath
+ val metadataDirectoryPath = CarbonTablePath.getMetadataPath(identifier.getTablePath)
val fileType = FileFactory.getFileType(metadataDirectoryPath)
if (!FileFactory.isFileExist(metadataDirectoryPath, fileType)) {
FileFactory.mkdirs(metadataDirectoryPath, fileType)
}
- GlobalDictionaryUtil
- .loadDefaultDictionaryValueForNewColumn(carbonTablePath,
- columnSchema,
- absoluteTableIdentifier,
- rawData)
+ GlobalDictionaryUtil.loadDefaultDictionaryValueForNewColumn(
+ columnSchema,
+ identifier,
+ rawData)
}
} catch {
case ex: Exception =>
http://git-wip-us.apache.org/repos/asf/carbondata/blob/2e1f2b95/integration/spark-common/src/main/scala/org/apache/carbondata/spark/rdd/CarbonGlobalDictionaryRDD.scala
----------------------------------------------------------------------
diff --git a/integration/spark-common/src/main/scala/org/apache/carbondata/spark/rdd/CarbonGlobalDictionaryRDD.scala b/integration/spark-common/src/main/scala/org/apache/carbondata/spark/rdd/CarbonGlobalDictionaryRDD.scala
index db29532..7acf4e2 100644
--- a/integration/spark-common/src/main/scala/org/apache/carbondata/spark/rdd/CarbonGlobalDictionaryRDD.scala
+++ b/integration/spark-common/src/main/scala/org/apache/carbondata/spark/rdd/CarbonGlobalDictionaryRDD.scala
@@ -40,10 +40,8 @@ import org.apache.carbondata.core.datastore.impl.FileFactory
import org.apache.carbondata.core.locks.{CarbonLockFactory, ICarbonLock, LockUsage}
import org.apache.carbondata.core.metadata.{AbsoluteTableIdentifier, CarbonTableIdentifier, ColumnIdentifier}
import org.apache.carbondata.core.metadata.schema.table.column.CarbonDimension
-import org.apache.carbondata.core.service.{CarbonCommonFactory, PathService}
import org.apache.carbondata.core.statusmanager.SegmentStatus
import org.apache.carbondata.core.util.{CarbonProperties, CarbonTimeStatisticsFactory, CarbonUtil}
-import org.apache.carbondata.core.util.path.{CarbonStorePath, CarbonTablePath}
import org.apache.carbondata.processing.loading.exception.NoRetryException
import org.apache.carbondata.processing.loading.model.CarbonLoadModel
import org.apache.carbondata.processing.util.CarbonLoaderUtil
@@ -348,10 +346,6 @@ class CarbonGlobalDictionaryGenerateRDD(
model.table,
model.columnIdentifier(split.index),
model.columnIdentifier(split.index).getDataType)
- val pathService: PathService = CarbonCommonFactory.getPathService
- val carbonTablePath: CarbonTablePath =
- pathService
- .getCarbonTablePath(model.table, dictionaryColumnUniqueIdentifier)
if (StringUtils.isNotBlank(model.hdfsTempLocation)) {
CarbonProperties.getInstance.addProperty(CarbonCommonConstants.HDFS_TEMP_LOCATION,
model.hdfsTempLocation)
http://git-wip-us.apache.org/repos/asf/carbondata/blob/2e1f2b95/integration/spark-common/src/main/scala/org/apache/carbondata/spark/util/CommonUtil.scala
----------------------------------------------------------------------
diff --git a/integration/spark-common/src/main/scala/org/apache/carbondata/spark/util/CommonUtil.scala b/integration/spark-common/src/main/scala/org/apache/carbondata/spark/util/CommonUtil.scala
index 94668bd..7815c99 100644
--- a/integration/spark-common/src/main/scala/org/apache/carbondata/spark/util/CommonUtil.scala
+++ b/integration/spark-common/src/main/scala/org/apache/carbondata/spark/util/CommonUtil.scala
@@ -50,7 +50,7 @@ import org.apache.carbondata.core.scan.partition.PartitionUtil
import org.apache.carbondata.core.statusmanager.{LoadMetadataDetails, SegmentStatusManager}
import org.apache.carbondata.core.util.{ByteUtil, CarbonProperties, CarbonUtil}
import org.apache.carbondata.core.util.comparator.Comparator
-import org.apache.carbondata.core.util.path.CarbonStorePath
+import org.apache.carbondata.core.util.path.CarbonTablePath
import org.apache.carbondata.processing.loading.csvinput.CSVInputFormat
import org.apache.carbondata.processing.loading.exception.CarbonDataLoadingException
import org.apache.carbondata.processing.loading.model.CarbonLoadModel
@@ -635,7 +635,7 @@ object CommonUtil {
def readLoadMetadataDetails(model: CarbonLoadModel): Unit = {
- val metadataPath = model.getCarbonDataLoadSchema.getCarbonTable.getMetaDataFilepath
+ val metadataPath = model.getCarbonDataLoadSchema.getCarbonTable.getMetadataPath
val details = SegmentStatusManager.readLoadMetadata(metadataPath)
model.setLoadMetadataDetails(new util.ArrayList[LoadMetadataDetails](details.toList.asJava))
}
@@ -866,20 +866,18 @@ object CommonUtil {
val fileType = FileFactory.getFileType(databaseLocation)
if (FileFactory.isFileExist(databaseLocation, fileType)) {
val file = FileFactory.getCarbonFile(databaseLocation, fileType)
- if (file.isDirectory) {
- val tableFolders = file.listFiles()
- tableFolders.foreach { tableFolder =>
- if (tableFolder.isDirectory) {
- val tablePath = databaseLocation +
- CarbonCommonConstants.FILE_SEPARATOR + tableFolder.getName
- val identifier =
- AbsoluteTableIdentifier.from(tablePath, dbName, tableFolder.getName)
- val carbonTablePath = CarbonStorePath.getCarbonTablePath(identifier)
- val tableStatusFile = carbonTablePath.getTableStatusFilePath
- if (FileFactory.isFileExist(tableStatusFile, fileType)) {
- val segmentStatusManager = new SegmentStatusManager(identifier)
- val carbonLock = segmentStatusManager.getTableStatusLock
- try {
+ if (file.isDirectory) {
+ val tableFolders = file.listFiles()
+ tableFolders.foreach { tableFolder =>
+ if (tableFolder.isDirectory) {
+ val tablePath = databaseLocation +
+ CarbonCommonConstants.FILE_SEPARATOR + tableFolder.getName
+ val identifier =
+ AbsoluteTableIdentifier.from(tablePath, dbName, tableFolder.getName)
+ val tableStatusFile =
+ CarbonTablePath.getTableStatusFilePath(tablePath)
+ if (FileFactory.isFileExist(tableStatusFile, fileType)) {
+ try {
val carbonTable = CarbonMetadata.getInstance
.getCarbonTable(identifier.getCarbonTableIdentifier.getTableUniqueName)
DataLoadingUtil.deleteLoadsAndUpdateMetadata(
http://git-wip-us.apache.org/repos/asf/carbondata/blob/2e1f2b95/integration/spark-common/src/main/scala/org/apache/carbondata/spark/util/DataLoadingUtil.scala
----------------------------------------------------------------------
diff --git a/integration/spark-common/src/main/scala/org/apache/carbondata/spark/util/DataLoadingUtil.scala b/integration/spark-common/src/main/scala/org/apache/carbondata/spark/util/DataLoadingUtil.scala
index 6767ef7..cf35c12 100644
--- a/integration/spark-common/src/main/scala/org/apache/carbondata/spark/util/DataLoadingUtil.scala
+++ b/integration/spark-common/src/main/scala/org/apache/carbondata/spark/util/DataLoadingUtil.scala
@@ -372,7 +372,7 @@ object DataLoadingUtil {
isForceDeletion: Boolean,
carbonTable: CarbonTable,
specs: util.List[PartitionSpec]): Unit = {
- if (isLoadDeletionRequired(carbonTable.getMetaDataFilepath)) {
+ if (isLoadDeletionRequired(carbonTable.getMetadataPath)) {
val absoluteTableIdentifier = carbonTable.getAbsoluteTableIdentifier
val (details, updationRequired) =
@@ -406,7 +406,7 @@ object DataLoadingUtil {
}
// read latest table status again.
val latestMetadata = SegmentStatusManager
- .readLoadMetadata(carbonTable.getMetaDataFilepath)
+ .readLoadMetadata(carbonTable.getMetadataPath)
// update the metadata details from old to new status.
val latestStatus = CarbonLoaderUtil
@@ -433,7 +433,7 @@ object DataLoadingUtil {
if (updationCompletionStaus) {
DeleteLoadFolders
.physicalFactAndMeasureMetadataDeletion(absoluteTableIdentifier,
- carbonTable.getMetaDataFilepath, isForceDeletion, specs)
+ carbonTable.getMetadataPath, isForceDeletion, specs)
}
}
}
@@ -449,7 +449,7 @@ object DataLoadingUtil {
absoluteTableIdentifier,
isForceDeletion,
details,
- carbonTable.getMetaDataFilepath
+ carbonTable.getMetadataPath
)
(details, isUpdationRequired)
}
http://git-wip-us.apache.org/repos/asf/carbondata/blob/2e1f2b95/integration/spark-common/src/main/scala/org/apache/carbondata/spark/util/GlobalDictionaryUtil.scala
----------------------------------------------------------------------
diff --git a/integration/spark-common/src/main/scala/org/apache/carbondata/spark/util/GlobalDictionaryUtil.scala b/integration/spark-common/src/main/scala/org/apache/carbondata/spark/util/GlobalDictionaryUtil.scala
index 9e1ece7..2bd4f45 100644
--- a/integration/spark-common/src/main/scala/org/apache/carbondata/spark/util/GlobalDictionaryUtil.scala
+++ b/integration/spark-common/src/main/scala/org/apache/carbondata/spark/util/GlobalDictionaryUtil.scala
@@ -52,7 +52,7 @@ import org.apache.carbondata.core.reader.CarbonDictionaryReader
import org.apache.carbondata.core.service.CarbonCommonFactory
import org.apache.carbondata.core.statusmanager.SegmentStatus
import org.apache.carbondata.core.util.{CarbonProperties, CarbonUtil, DataTypeUtil}
-import org.apache.carbondata.core.util.path.{CarbonStorePath, CarbonTablePath}
+import org.apache.carbondata.core.util.path.CarbonTablePath
import org.apache.carbondata.core.writer.CarbonDictionaryWriter
import org.apache.carbondata.processing.exception.DataLoadingException
import org.apache.carbondata.processing.loading.csvinput.{CSVInputFormat, StringArrayWritable}
@@ -308,7 +308,7 @@ object GlobalDictionaryUtil {
}
val primDimensions = primDimensionsBuffer.map { x => x }.toArray
val dictDetail = CarbonSparkFactory.getDictionaryDetailService.
- getDictionaryDetail(dictFolderPath, primDimensions, table, carbonLoadModel.getTablePath)
+ getDictionaryDetail(dictFolderPath, primDimensions, carbonLoadModel.getTablePath)
val dictFilePaths = dictDetail.dictFilePaths
val dictFileExists = dictDetail.dictFileExists
val columnIdentifier = dictDetail.columnIdentifiers
@@ -398,10 +398,6 @@ object GlobalDictionaryUtil {
}
}
- // Hack for spark2 integration
- var updateTableMetadataFunc: (CarbonLoadModel, SQLContext, DictionaryLoadModel,
- Array[CarbonDimension]) => Unit = _
-
/**
* check whether global dictionary have been generated successfully or not
*
@@ -705,10 +701,7 @@ object GlobalDictionaryUtil {
try {
val carbonTable = carbonLoadModel.getCarbonDataLoadSchema.getCarbonTable
val carbonTableIdentifier = carbonTable.getAbsoluteTableIdentifier.getCarbonTableIdentifier
- // create dictionary folder if not exists
- val tablePath = carbonLoadModel.getTablePath
- val carbonTablePath = CarbonStorePath.getCarbonTablePath(tablePath, carbonTableIdentifier)
- val dictfolderPath = carbonTablePath.getMetadataDirectoryPath
+ val dictfolderPath = CarbonTablePath.getMetadataPath(carbonLoadModel.getTablePath)
// columns which need to generate global dictionary file
val dimensions = carbonTable.getDimensionByTableName(
carbonTable.getTableName).asScala.toArray
@@ -845,12 +838,11 @@ object GlobalDictionaryUtil {
* This method will write dictionary file, sortindex file and dictionary meta for new dictionary
* column with default value
*
- * @param carbonTablePath
* @param columnSchema
* @param absoluteTableIdentifier
* @param defaultValue
*/
- def loadDefaultDictionaryValueForNewColumn(carbonTablePath: CarbonTablePath,
+ def loadDefaultDictionaryValueForNewColumn(
columnSchema: ColumnSchema,
absoluteTableIdentifier: AbsoluteTableIdentifier,
defaultValue: String): Unit = {
http://git-wip-us.apache.org/repos/asf/carbondata/blob/2e1f2b95/integration/spark-common/src/main/scala/org/apache/spark/sql/execution/command/carbonTableSchemaCommon.scala
----------------------------------------------------------------------
diff --git a/integration/spark-common/src/main/scala/org/apache/spark/sql/execution/command/carbonTableSchemaCommon.scala b/integration/spark-common/src/main/scala/org/apache/spark/sql/execution/command/carbonTableSchemaCommon.scala
index 20d3032..71ce2c6 100644
--- a/integration/spark-common/src/main/scala/org/apache/spark/sql/execution/command/carbonTableSchemaCommon.scala
+++ b/integration/spark-common/src/main/scala/org/apache/spark/sql/execution/command/carbonTableSchemaCommon.scala
@@ -33,12 +33,14 @@ import org.apache.carbondata.core.datamap.Segment
import org.apache.carbondata.core.indexstore.PartitionSpec
import org.apache.carbondata.core.metadata.CarbonTableIdentifier
import org.apache.carbondata.core.metadata.SegmentFileStore.SegmentFile
+import org.apache.carbondata.core.metadata.{AbsoluteTableIdentifier, CarbonTableIdentifier}
import org.apache.carbondata.core.metadata.datatype.{DataType, DataTypes, DecimalType}
import org.apache.carbondata.core.metadata.encoder.Encoding
import org.apache.carbondata.core.metadata.schema._
import org.apache.carbondata.core.metadata.schema.table.{CarbonTable, RelationIdentifier, TableInfo, TableSchema}
import org.apache.carbondata.core.metadata.schema.table.column.{ColumnSchema, ParentColumnTableRelation}
import org.apache.carbondata.core.service.CarbonCommonFactory
+import org.apache.carbondata.core.service.impl.ColumnUniqueIdGenerator
import org.apache.carbondata.core.statusmanager.{LoadMetadataDetails, SegmentUpdateStatusManager}
import org.apache.carbondata.core.util.DataTypeUtil
import org.apache.carbondata.core.util.path.CarbonTablePath
@@ -208,8 +210,7 @@ class AlterTableColumnSchemaGenerator(
alterTableModel: AlterTableAddColumnsModel,
dbName: String,
tableInfo: TableInfo,
- carbonTablePath: CarbonTablePath,
- tableIdentifier: CarbonTableIdentifier,
+ tableIdentifier: AbsoluteTableIdentifier,
sc: SparkContext) {
val LOGGER = LogServiceFactory.getLogService(this.getClass.getName)
@@ -370,7 +371,7 @@ object TableNewProcessor {
encoders.add(Encoding.DIRECT_DICTIONARY)
}
columnSchema.setEncodingList(encoders)
- val colUniqueIdGenerator = CarbonCommonFactory.getColumnUniqueIdGenerator
+ val colUniqueIdGenerator = ColumnUniqueIdGenerator.getInstance
val columnUniqueId = colUniqueIdGenerator.generateUniqueId(columnSchema)
columnSchema.setColumnUniqueId(columnUniqueId)
columnSchema.setColumnReferenceId(columnUniqueId)
@@ -434,7 +435,7 @@ class TableNewProcessor(cm: TableModel) {
}
}
columnSchema.setEncodingList(encoders)
- val colUniqueIdGenerator = CarbonCommonFactory.getColumnUniqueIdGenerator
+ val colUniqueIdGenerator = ColumnUniqueIdGenerator.getInstance
val columnUniqueId = colUniqueIdGenerator.generateUniqueId(columnSchema)
columnSchema.setColumnUniqueId(columnUniqueId)
columnSchema.setColumnReferenceId(columnUniqueId)
http://git-wip-us.apache.org/repos/asf/carbondata/blob/2e1f2b95/integration/spark-common/src/main/scala/org/apache/spark/util/PartitionUtils.scala
----------------------------------------------------------------------
diff --git a/integration/spark-common/src/main/scala/org/apache/spark/util/PartitionUtils.scala b/integration/spark-common/src/main/scala/org/apache/spark/util/PartitionUtils.scala
index 3c871db..1656efa 100644
--- a/integration/spark-common/src/main/scala/org/apache/spark/util/PartitionUtils.scala
+++ b/integration/spark-common/src/main/scala/org/apache/spark/util/PartitionUtils.scala
@@ -176,8 +176,6 @@ object PartitionUtils {
getPartitionBlockList(identifier, segmentId, partitionIds, oldPartitionIds,
partitionInfo, carbonLoadModel.getCarbonDataLoadSchema.getCarbonTable).asScala
val pathList: util.List[String] = new util.ArrayList[String]()
- val carbonTableIdentifier = new CarbonTableIdentifier(dbName, tableName, "")
- val carbonTablePath = new CarbonTablePath(carbonTableIdentifier, tablePath)
tableBlockInfoList.foreach{ tableBlockInfo =>
val path = tableBlockInfo.getFilePath
val timestamp = CarbonTablePath.DataFileUtil.getTimeStampFromFileName(path)
@@ -190,8 +188,8 @@ object PartitionUtils {
val batchNo = CarbonTablePath.DataFileUtil.getBatchNoFromTaskNo(taskNo)
val taskId = CarbonTablePath.DataFileUtil.getTaskIdFromTaskNo(taskNo)
val bucketNumber = CarbonTablePath.DataFileUtil.getBucketNo(path)
- val indexFilePath = carbonTablePath.getCarbonIndexFilePath(
- String.valueOf(taskId), segmentId, batchNo, String.valueOf(bucketNumber),
+ val indexFilePath = CarbonTablePath.getCarbonIndexFilePath(
+ tablePath, String.valueOf(taskId), segmentId, batchNo, String.valueOf(bucketNumber),
timestamp, version)
// indexFilePath could be duplicated when multiple data file related to one index file
if (indexFilePath != null && !pathList.contains(indexFilePath)) {