This is an automated email from the ASF dual-hosted git repository.

jackie pushed a commit to branch master
in repository https://gitbox.apache.org/repos/asf/pinot.git


The following commit(s) were added to refs/heads/master by this push:
     new 71b6890  Check schema backward-compatibility when updating schema 
through addSchema with override (#7374)
71b6890 is described below

commit 71b68900913d2382e9c72c0c93b46acd2d3de2ff
Author: Xiaotian (Jackie) Jiang <17555551+jackie-ji...@users.noreply.github.com>
AuthorDate: Tue Aug 31 13:37:41 2021 -0700

    Check schema backward-compatibility when updating schema through addSchema 
with override (#7374)
    
    Currently addSchema() with override on will by-pass the 
backward-compatibility check, which can cause bad schema change. Fix it by 
adding the check when override is triggered.
    Also refined the error code when schema cannot be updated.
---
 .../exception/SchemaAlreadyExistsException.java    |  26 +++++
 .../api/resources/PinotSchemaRestletResource.java  |  11 +-
 .../helix/ControllerRequestURLBuilder.java         |   4 +
 .../helix/core/PinotHelixResourceManager.java      |  81 +++++++------
 .../api/PinotSchemaRestletResourceTest.java        | 128 +++++++++++++++------
 .../pinot/controller/helix/ControllerTest.java     |  25 +---
 .../tests/OfflineClusterIntegrationTest.java       |   3 +
 7 files changed, 188 insertions(+), 90 deletions(-)

diff --git 
a/pinot-common/src/main/java/org/apache/pinot/common/exception/SchemaAlreadyExistsException.java
 
b/pinot-common/src/main/java/org/apache/pinot/common/exception/SchemaAlreadyExistsException.java
new file mode 100644
index 0000000..e1dd430
--- /dev/null
+++ 
b/pinot-common/src/main/java/org/apache/pinot/common/exception/SchemaAlreadyExistsException.java
@@ -0,0 +1,26 @@
+/**
+ * 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.pinot.common.exception;
+
+public class SchemaAlreadyExistsException extends Exception {
+
+  public SchemaAlreadyExistsException(String message) {
+    super(message);
+  }
+}
diff --git 
a/pinot-controller/src/main/java/org/apache/pinot/controller/api/resources/PinotSchemaRestletResource.java
 
b/pinot-controller/src/main/java/org/apache/pinot/controller/api/resources/PinotSchemaRestletResource.java
index d6eded5..d795aad 100644
--- 
a/pinot-controller/src/main/java/org/apache/pinot/controller/api/resources/PinotSchemaRestletResource.java
+++ 
b/pinot-controller/src/main/java/org/apache/pinot/controller/api/resources/PinotSchemaRestletResource.java
@@ -43,6 +43,7 @@ import javax.ws.rs.core.Context;
 import javax.ws.rs.core.HttpHeaders;
 import javax.ws.rs.core.MediaType;
 import javax.ws.rs.core.Response;
+import org.apache.pinot.common.exception.SchemaAlreadyExistsException;
 import org.apache.pinot.common.exception.SchemaBackwardIncompatibleException;
 import org.apache.pinot.common.exception.SchemaNotFoundException;
 import org.apache.pinot.common.exception.TableNotFoundException;
@@ -180,7 +181,7 @@ public class PinotSchemaRestletResource {
   @ApiOperation(value = "Add a new schema", notes = "Adds a new schema")
   @ApiResponses(value = {
       @ApiResponse(code = 200, message = "Successfully created schema"),
-      @ApiResponse(code = 404, message = "Schema not found"),
+      @ApiResponse(code = 409, message = "Schema already exists"),
       @ApiResponse(code = 400, message = "Missing or invalid request body"),
       @ApiResponse(code = 500, message = "Internal error")
   })
@@ -202,7 +203,7 @@ public class PinotSchemaRestletResource {
   @ApiOperation(value = "Add a new schema", notes = "Adds a new schema")
   @ApiResponses(value = {
       @ApiResponse(code = 200, message = "Successfully created schema"),
-      @ApiResponse(code = 404, message = "Schema not found"),
+      @ApiResponse(code = 409, message = "Schema already exists"),
       @ApiResponse(code = 400, message = "Missing or invalid request body"),
       @ApiResponse(code = 500, message = "Internal error")
   })
@@ -282,6 +283,12 @@ public class PinotSchemaRestletResource {
       _metadataEventNotifierFactory.create().notifyOnSchemaEvents(schema, 
SchemaEventType.CREATE);
 
       return new SuccessResponse(schemaName + " successfully added");
+    } catch (SchemaAlreadyExistsException e) {
+      
_controllerMetrics.addMeteredGlobalValue(ControllerMeter.CONTROLLER_SCHEMA_UPLOAD_ERROR,
 1L);
+      throw new ControllerApplicationException(LOGGER, e.getMessage(), 
Response.Status.CONFLICT, e);
+    } catch (SchemaBackwardIncompatibleException e) {
+      
_controllerMetrics.addMeteredGlobalValue(ControllerMeter.CONTROLLER_SCHEMA_UPLOAD_ERROR,
 1L);
+      throw new ControllerApplicationException(LOGGER, e.getMessage(), 
Response.Status.BAD_REQUEST, e);
     } catch (Exception e) {
       
_controllerMetrics.addMeteredGlobalValue(ControllerMeter.CONTROLLER_SCHEMA_UPLOAD_ERROR,
 1L);
       throw new ControllerApplicationException(LOGGER, String.format("Failed 
to add new schema %s.", schemaName),
diff --git 
a/pinot-controller/src/main/java/org/apache/pinot/controller/helix/ControllerRequestURLBuilder.java
 
b/pinot-controller/src/main/java/org/apache/pinot/controller/helix/ControllerRequestURLBuilder.java
index 5a4b371..f4ae93b 100644
--- 
a/pinot-controller/src/main/java/org/apache/pinot/controller/helix/ControllerRequestURLBuilder.java
+++ 
b/pinot-controller/src/main/java/org/apache/pinot/controller/helix/ControllerRequestURLBuilder.java
@@ -244,6 +244,10 @@ public class ControllerRequestURLBuilder {
     return StringUtil.join("/", _baseUrl, "schemas", schemaName);
   }
 
+  public String forSchemaDelete(String schemaName) {
+    return StringUtil.join("/", _baseUrl, "schemas", schemaName);
+  }
+
   public String forTableConfigsCreate() {
     return StringUtil.join("/", _baseUrl, "tableConfigs");
   }
diff --git 
a/pinot-controller/src/main/java/org/apache/pinot/controller/helix/core/PinotHelixResourceManager.java
 
b/pinot-controller/src/main/java/org/apache/pinot/controller/helix/core/PinotHelixResourceManager.java
index 92f6030..f094392 100644
--- 
a/pinot-controller/src/main/java/org/apache/pinot/controller/helix/core/PinotHelixResourceManager.java
+++ 
b/pinot-controller/src/main/java/org/apache/pinot/controller/helix/core/PinotHelixResourceManager.java
@@ -73,6 +73,7 @@ import 
org.apache.pinot.common.assignment.InstanceAssignmentConfigUtils;
 import org.apache.pinot.common.assignment.InstancePartitions;
 import org.apache.pinot.common.assignment.InstancePartitionsUtils;
 import org.apache.pinot.common.exception.InvalidConfigException;
+import org.apache.pinot.common.exception.SchemaAlreadyExistsException;
 import org.apache.pinot.common.exception.SchemaBackwardIncompatibleException;
 import org.apache.pinot.common.exception.SchemaNotFoundException;
 import org.apache.pinot.common.exception.TableNotFoundException;
@@ -90,7 +91,6 @@ import 
org.apache.pinot.common.metadata.instance.InstanceZKMetadata;
 import org.apache.pinot.common.metadata.segment.SegmentZKMetadata;
 import org.apache.pinot.common.minion.MinionTaskMetadataUtils;
 import org.apache.pinot.common.utils.HashUtil;
-import org.apache.pinot.common.utils.SchemaUtils;
 import org.apache.pinot.common.utils.config.InstanceUtils;
 import org.apache.pinot.common.utils.config.TableConfigUtils;
 import org.apache.pinot.common.utils.config.TagNameUtils;
@@ -1036,54 +1036,45 @@ public class PinotHelixResourceManager {
     return 
HelixHelper.getBrokerInstanceConfigsForTenant(HelixHelper.getInstanceConfigs(_helixZkManager),
 tenantName);
   }
 
-  /**
+  /*
    * API 2.0
    */
 
-  /**
+  /*
    * Schema APIs
    */
-  public void addSchema(Schema schema, boolean override) {
-    ZNRecord record = SchemaUtils.toZNRecord(schema);
-    String schemaName = schema.getSchemaName();
-    Schema oldSchema = ZKMetadataProvider.getSchema(_propertyStore, 
schemaName);
 
-    if (oldSchema != null && !override) {
-      throw new RuntimeException(String.format("Schema %s exists. Not 
overriding it as requested.", schemaName));
-    }
+  public void addSchema(Schema schema, boolean override)
+      throws SchemaAlreadyExistsException, SchemaBackwardIncompatibleException 
{
+    String schemaName = schema.getSchemaName();
+    LOGGER.info("Adding schema: {} with override: {}", schemaName, override);
 
-    if (schema.equals(oldSchema)) {
-      LOGGER.info("New schema is the same with the existing schema. Not 
updating schema " + schemaName);
-      return;
+    Schema oldSchema = ZKMetadataProvider.getSchema(_propertyStore, 
schemaName);
+    if (oldSchema != null) {
+      // Update existing schema
+      if (override) {
+        updateSchema(schema, oldSchema);
+      } else {
+        throw new SchemaAlreadyExistsException(String.format("Schema: %s 
already exists", schemaName));
+      }
+    } else {
+      // Add new schema
+      ZKMetadataProvider.setSchema(_propertyStore, schema);
+      LOGGER.info("Added schema: {}", schemaName);
     }
-
-    PinotHelixPropertyStoreZnRecordProvider propertyStoreHelper =
-        PinotHelixPropertyStoreZnRecordProvider.forSchema(_propertyStore);
-    propertyStoreHelper.set(schemaName, record);
   }
 
   public void updateSchema(Schema schema, boolean reload)
       throws SchemaNotFoundException, SchemaBackwardIncompatibleException, 
TableNotFoundException {
     String schemaName = schema.getSchemaName();
-    Schema oldSchema = ZKMetadataProvider.getSchema(_propertyStore, 
schemaName);
+    LOGGER.info("Updating schema: {} with reload: {}", schemaName, reload);
 
+    Schema oldSchema = ZKMetadataProvider.getSchema(_propertyStore, 
schemaName);
     if (oldSchema == null) {
-      throw new SchemaNotFoundException(String.format("Schema %s did not 
exist.", schemaName));
+      throw new SchemaNotFoundException(String.format("Schema: %s does not 
exist", schemaName));
     }
 
-    schema.updateBooleanFieldsIfNeeded(oldSchema);
-
-    if (schema.equals(oldSchema)) {
-      LOGGER.info("New schema is the same with the existing schema. Not 
updating schema " + schemaName);
-      return;
-    }
-
-    if (!schema.isBackwardCompatibleWith(oldSchema)) {
-      throw new SchemaBackwardIncompatibleException(
-          String.format("New schema %s is not backward compatible with the 
current schema", schemaName));
-    }
-
-    ZKMetadataProvider.setSchema(_propertyStore, schema);
+    updateSchema(schema, oldSchema);
 
     if (reload) {
       LOGGER.info("Reloading tables with name: {}", schemaName);
@@ -1095,15 +1086,39 @@ public class PinotHelixResourceManager {
   }
 
   /**
+   * Helper method to update the schema, or throw 
SchemaBackwardIncompatibleException when the new schema is not
+   * backward-compatible with the existing schema.
+   */
+  private void updateSchema(Schema schema, Schema oldSchema)
+      throws SchemaBackwardIncompatibleException {
+    String schemaName = schema.getSchemaName();
+    schema.updateBooleanFieldsIfNeeded(oldSchema);
+    if (schema.equals(oldSchema)) {
+      LOGGER.info("New schema: {} is the same as the existing schema, not 
updating it", schemaName);
+      return;
+    }
+    if (!schema.isBackwardCompatibleWith(oldSchema)) {
+      // TODO: Add the reason of the incompatibility
+      throw new SchemaBackwardIncompatibleException(
+          String.format("New schema: %s is not backward-compatible with the 
existing schema", schemaName));
+    }
+    ZKMetadataProvider.setSchema(_propertyStore, schema);
+    LOGGER.info("Updated schema: {}", schemaName);
+  }
+
+  /**
    * Delete the given schema.
    * @param schema The schema to be deleted.
    * @return True on success, false otherwise.
    */
   public boolean deleteSchema(Schema schema) {
     if (schema != null) {
-      String propertyStorePath = 
ZKMetadataProvider.constructPropertyStorePathForSchema(schema.getSchemaName());
+      String schemaName = schema.getSchemaName();
+      LOGGER.info("Deleting schema: {}", schemaName);
+      String propertyStorePath = 
ZKMetadataProvider.constructPropertyStorePathForSchema(schemaName);
       if (_propertyStore.exists(propertyStorePath, AccessOption.PERSISTENT)) {
         _propertyStore.remove(propertyStorePath, AccessOption.PERSISTENT);
+        LOGGER.info("Deleted schema: {}", schemaName);
         return true;
       }
     }
diff --git 
a/pinot-controller/src/test/java/org/apache/pinot/controller/api/PinotSchemaRestletResourceTest.java
 
b/pinot-controller/src/test/java/org/apache/pinot/controller/api/PinotSchemaRestletResourceTest.java
index 1888193..2d51a18 100644
--- 
a/pinot-controller/src/test/java/org/apache/pinot/controller/api/PinotSchemaRestletResourceTest.java
+++ 
b/pinot-controller/src/test/java/org/apache/pinot/controller/api/PinotSchemaRestletResourceTest.java
@@ -25,7 +25,7 @@ import org.apache.commons.httpclient.methods.PostMethod;
 import org.apache.commons.httpclient.methods.PutMethod;
 import org.apache.pinot.controller.ControllerTestUtils;
 import org.apache.pinot.spi.data.DimensionFieldSpec;
-import org.apache.pinot.spi.data.FieldSpec;
+import org.apache.pinot.spi.data.FieldSpec.DataType;
 import org.apache.pinot.spi.data.Schema;
 import org.testng.Assert;
 import org.testng.annotations.AfterClass;
@@ -93,48 +93,106 @@ public class PinotSchemaRestletResourceTest {
       throws IOException {
     String schemaName = "testSchema";
     Schema schema = ControllerTestUtils.createDummySchema(schemaName);
-    String url = 
ControllerTestUtils.getControllerRequestURLBuilder().forSchemaCreate();
-    PostMethod postMethod = ControllerTestUtils.sendMultipartPostRequest(url, 
schema.toSingleLineJsonString());
+
+    // Add the schema
+    String addSchemaUrl = 
ControllerTestUtils.getControllerRequestURLBuilder().forSchemaCreate();
+    PostMethod postMethod = 
ControllerTestUtils.sendMultipartPostRequest(addSchemaUrl, 
schema.toSingleLineJsonString());
     Assert.assertEquals(postMethod.getStatusCode(), 200);
 
-    schema.addField(new DimensionFieldSpec("NewColumn", 
FieldSpec.DataType.STRING, true));
-    postMethod = ControllerTestUtils.sendMultipartPostRequest(url, 
schema.toSingleLineJsonString());
+    // Add a new column
+    DimensionFieldSpec newColumnFieldSpec = new 
DimensionFieldSpec("newColumn", DataType.STRING, true);
+    schema.addField(newColumnFieldSpec);
+
+    // Update the schema with addSchema api and override off
+    postMethod =
+        ControllerTestUtils.sendMultipartPostRequest(addSchemaUrl + 
"?override=false", schema.toSingleLineJsonString());
+    Assert.assertEquals(postMethod.getStatusCode(), 409);
+
+    // Update the schema with addSchema api and override on
+    postMethod = ControllerTestUtils.sendMultipartPostRequest(addSchemaUrl, 
schema.toSingleLineJsonString());
     Assert.assertEquals(postMethod.getStatusCode(), 200);
 
-    String schemaStr = ControllerTestUtils
-        
.sendGetRequest(ControllerTestUtils.getControllerRequestURLBuilder().forSchemaGet(schemaName));
-    Schema readSchema = Schema.fromString(schemaStr);
-    Schema inputSchema = Schema.fromString(schema.toSingleLineJsonString());
-    Assert.assertEquals(readSchema, inputSchema);
-    Assert.assertTrue(readSchema.getFieldSpecMap().containsKey("NewColumn"));
-
-    final String yetAnotherColumn = "YetAnotherColumn";
-    
Assert.assertFalse(readSchema.getFieldSpecMap().containsKey(yetAnotherColumn));
-    schema.addField(new DimensionFieldSpec(yetAnotherColumn, 
FieldSpec.DataType.STRING, true));
-    PutMethod putMethod = ControllerTestUtils
-        
.sendMultipartPutRequest(ControllerTestUtils.getControllerRequestURLBuilder().forSchemaUpdate(schemaName),
-            schema.toSingleLineJsonString());
+    // Get the schema and verify the new column exists
+    String getSchemaUrl = 
ControllerTestUtils.getControllerRequestURLBuilder().forSchemaGet(schemaName);
+    Schema remoteSchema = 
Schema.fromString(ControllerTestUtils.sendGetRequest(getSchemaUrl));
+    Assert.assertEquals(remoteSchema, schema);
+    Assert.assertTrue(remoteSchema.hasColumn(newColumnFieldSpec.getName()));
+
+    // Add another new column
+    DimensionFieldSpec newColumnFieldSpec2 = new 
DimensionFieldSpec("newColumn2", DataType.STRING, true);
+    schema.addField(newColumnFieldSpec2);
+
+    // Update the schema with updateSchema api
+    String updateSchemaUrl = 
ControllerTestUtils.getControllerRequestURLBuilder().forSchemaUpdate(schemaName);
+    PutMethod putMethod = 
ControllerTestUtils.sendMultipartPutRequest(updateSchemaUrl, 
schema.toSingleLineJsonString());
     Assert.assertEquals(putMethod.getStatusCode(), 200);
-    // verify some more...
-    schemaStr = ControllerTestUtils
-        
.sendGetRequest(ControllerTestUtils.getControllerRequestURLBuilder().forSchemaGet(schemaName));
-    readSchema = Schema.fromString(schemaStr);
-    inputSchema = Schema.fromString(schema.toSingleLineJsonString());
-    Assert.assertEquals(readSchema, inputSchema);
-    
Assert.assertTrue(readSchema.getFieldSpecMap().containsKey(yetAnotherColumn));
-
-    // error cases
-    putMethod = ControllerTestUtils
-        
.sendMultipartPutRequest(ControllerTestUtils.getControllerRequestURLBuilder().forSchemaUpdate(schemaName),
-            schema.toSingleLineJsonString().substring(1));
-    // invalid json
+
+    // Get the schema and verify both the new columns exist
+    remoteSchema = 
Schema.fromString(ControllerTestUtils.sendGetRequest(getSchemaUrl));
+    Assert.assertEquals(remoteSchema, schema);
+    Assert.assertTrue(remoteSchema.hasColumn(newColumnFieldSpec.getName()));
+    Assert.assertTrue(remoteSchema.hasColumn(newColumnFieldSpec2.getName()));
+
+    // Change the column data type - backward-incompatible change
+    newColumnFieldSpec.setDataType(DataType.INT);
+
+    // Update the schema with addSchema api and override on
+    postMethod = ControllerTestUtils.sendMultipartPostRequest(addSchemaUrl, 
schema.toSingleLineJsonString());
+    Assert.assertEquals(postMethod.getStatusCode(), 400);
+
+    // Update the schema with updateSchema api
+    putMethod = ControllerTestUtils.sendMultipartPutRequest(updateSchemaUrl, 
schema.toSingleLineJsonString());
     Assert.assertEquals(putMethod.getStatusCode(), 400);
 
-    schema.setSchemaName("differentSchemaName");
-    putMethod = ControllerTestUtils
-        
.sendMultipartPutRequest(ControllerTestUtils.getControllerRequestURLBuilder().forSchemaUpdate(schemaName),
-            schema.toSingleLineJsonString());
+    // Change the column data type from STRING to BOOLEAN
+    newColumnFieldSpec.setDataType(DataType.BOOLEAN);
+
+    // Update the schema with addSchema api and override on
+    postMethod = ControllerTestUtils.sendMultipartPostRequest(addSchemaUrl, 
schema.toSingleLineJsonString());
+    Assert.assertEquals(postMethod.getStatusCode(), 200);
+
+    // Change another column data type from STRING to BOOLEAN
+    newColumnFieldSpec2.setDataType(DataType.BOOLEAN);
+
+    // Update the schema with updateSchema api
+    putMethod = ControllerTestUtils.sendMultipartPutRequest(updateSchemaUrl, 
schema.toSingleLineJsonString());
+    Assert.assertEquals(putMethod.getStatusCode(), 200);
+
+    // Get the schema and verify the data types are not changed
+    remoteSchema = 
Schema.fromString(ControllerTestUtils.sendGetRequest(getSchemaUrl));
+    
Assert.assertEquals(remoteSchema.getFieldSpecFor(newColumnFieldSpec.getName()).getDataType(),
 DataType.STRING);
+    
Assert.assertEquals(remoteSchema.getFieldSpecFor(newColumnFieldSpec2.getName()).getDataType(),
 DataType.STRING);
+
+    // Add a new BOOLEAN column
+    DimensionFieldSpec newColumnFieldSpec3 = new 
DimensionFieldSpec("newColumn3", DataType.BOOLEAN, true);
+    schema.addField(newColumnFieldSpec3);
+
+    // Update the schema with updateSchema api
+    putMethod = ControllerTestUtils.sendMultipartPutRequest(updateSchemaUrl, 
schema.toSingleLineJsonString());
+    Assert.assertEquals(putMethod.getStatusCode(), 200);
+
+    // Get the schema and verify the new column has BOOLEAN data type
+    remoteSchema = 
Schema.fromString(ControllerTestUtils.sendGetRequest(getSchemaUrl));
+    
Assert.assertEquals(remoteSchema.getFieldSpecFor(newColumnFieldSpec3.getName()).getDataType(),
 DataType.BOOLEAN);
+
+    // Post invalid schema string
+    String invalidSchemaString = schema.toSingleLineJsonString().substring(1);
+    postMethod = ControllerTestUtils.sendMultipartPostRequest(addSchemaUrl, 
invalidSchemaString);
+    Assert.assertEquals(postMethod.getStatusCode(), 400);
+    putMethod = ControllerTestUtils.sendMultipartPutRequest(updateSchemaUrl, 
invalidSchemaString);
     Assert.assertEquals(putMethod.getStatusCode(), 400);
+
+    // Update schema with non-matching schema name
+    String newSchemaName = "newSchemaName";
+    schema.setSchemaName(newSchemaName);
+    putMethod = ControllerTestUtils.sendMultipartPutRequest(updateSchemaUrl, 
schema.toSingleLineJsonString());
+    Assert.assertEquals(putMethod.getStatusCode(), 400);
+
+    // Update non-existing schema
+    putMethod = ControllerTestUtils.sendMultipartPutRequest(
+        
ControllerTestUtils.getControllerRequestURLBuilder().forSchemaUpdate(newSchemaName),
+        schema.toSingleLineJsonString());
+    Assert.assertEquals(putMethod.getStatusCode(), 404);
   }
 
   @AfterClass
diff --git 
a/pinot-controller/src/test/java/org/apache/pinot/controller/helix/ControllerTest.java
 
b/pinot-controller/src/test/java/org/apache/pinot/controller/helix/ControllerTest.java
index 898e38c..2111899 100644
--- 
a/pinot-controller/src/test/java/org/apache/pinot/controller/helix/ControllerTest.java
+++ 
b/pinot-controller/src/test/java/org/apache/pinot/controller/helix/ControllerTest.java
@@ -71,10 +71,6 @@ import org.apache.pinot.spi.config.table.TableConfig;
 import org.apache.pinot.spi.config.table.TableType;
 import org.apache.pinot.spi.config.tenant.Tenant;
 import org.apache.pinot.spi.config.tenant.TenantRole;
-import org.apache.pinot.spi.data.DateTimeFieldSpec;
-import org.apache.pinot.spi.data.DimensionFieldSpec;
-import org.apache.pinot.spi.data.FieldSpec;
-import org.apache.pinot.spi.data.MetricFieldSpec;
 import org.apache.pinot.spi.data.Schema;
 import org.apache.pinot.spi.env.PinotConfiguration;
 import org.apache.pinot.spi.utils.CommonConstants;
@@ -499,22 +495,6 @@ public abstract class ControllerTest {
     }
   }
 
-  protected Schema createDummySchema(String tableName) {
-    Schema schema = new Schema();
-    schema.setSchemaName(tableName);
-    schema.addField(new DimensionFieldSpec("dimA", FieldSpec.DataType.STRING, 
true, ""));
-    schema.addField(new DimensionFieldSpec("dimB", FieldSpec.DataType.STRING, 
true, 0));
-    schema.addField(new MetricFieldSpec("metricA", FieldSpec.DataType.INT, 0));
-    schema.addField(new MetricFieldSpec("metricB", FieldSpec.DataType.DOUBLE, 
-1));
-    schema.addField(new DateTimeFieldSpec("timeColumn", 
FieldSpec.DataType.LONG, "1:MILLISECONDS:EPOCH", "1:DAYS"));
-    return schema;
-  }
-
-  protected void addDummySchema(String tableName)
-      throws IOException {
-    addSchema(createDummySchema(tableName));
-  }
-
   /**
    * Add a schema to the controller.
    */
@@ -531,6 +511,11 @@ public abstract class ControllerTest {
     return schema;
   }
 
+  protected void deleteSchema(String schemaName)
+      throws IOException {
+    
sendDeleteRequest(_controllerRequestURLBuilder.forSchemaDelete(schemaName));
+  }
+
   protected void addTableConfig(TableConfig tableConfig)
       throws IOException {
     sendPostRequest(_controllerRequestURLBuilder.forTableCreate(), 
tableConfig.toJsonString());
diff --git 
a/pinot-integration-tests/src/test/java/org/apache/pinot/integration/tests/OfflineClusterIntegrationTest.java
 
b/pinot-integration-tests/src/test/java/org/apache/pinot/integration/tests/OfflineClusterIntegrationTest.java
index 9d1df36..ad38c9c 100644
--- 
a/pinot-integration-tests/src/test/java/org/apache/pinot/integration/tests/OfflineClusterIntegrationTest.java
+++ 
b/pinot-integration-tests/src/test/java/org/apache/pinot/integration/tests/OfflineClusterIntegrationTest.java
@@ -764,6 +764,9 @@ public class OfflineClusterIntegrationTest extends 
BaseClusterIntegrationTestSet
     TableConfig tableConfig = getOfflineTableConfig();
     tableConfig.setIngestionConfig(null);
     updateTableConfig(tableConfig);
+
+    // Need to first delete then add the schema because removing columns is 
backward-incompatible change
+    deleteSchema(getSchemaName());
     _schemaFileName = SCHEMA_FILE_NAME_WITH_MISSING_COLUMNS;
     addSchema(createSchema());
 

---------------------------------------------------------------------
To unsubscribe, e-mail: commits-unsubscr...@pinot.apache.org
For additional commands, e-mail: commits-h...@pinot.apache.org

Reply via email to