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 9873ed9b96 add a experiment API for upsert heap memory estimation 
(#8355)
9873ed9b96 is described below

commit 9873ed9b967b25b64b94492df1490797a768fc2c
Author: deemoliu <qiao...@uber.com>
AuthorDate: Wed May 4 09:40:52 2022 -0700

    add a experiment API for upsert heap memory estimation (#8355)
    
    Added a http endpoint for estimate the heap usage for Pinot upsert table  
per partition based on PK cardinality.
    - POST /upsert/estimateHeapUsage
---
 .../pinot/controller/api/resources/Constants.java  |   1 +
 .../api/resources/PinotUpsertRestletResource.java  | 153 +++++++++++++++++++++
 .../controller/helix/ControllerRequestClient.java  |  10 +-
 .../pinot/controller/ControllerTestUtils.java      |   6 +
 .../api/PinotUpsertRestletResourceTest.java        |  78 +++++++++++
 .../memory_estimation/schema-for-upsert.json       |  96 +++++++++++++
 .../memory_estimation/table-config-for-upsert.json |  53 +++++++
 .../apache/pinot/spi/config/table/ColumnStats.java |  89 ++++++++++++
 .../utils/builder/ControllerRequestURLBuilder.java |   5 +
 9 files changed, 486 insertions(+), 5 deletions(-)

diff --git 
a/pinot-controller/src/main/java/org/apache/pinot/controller/api/resources/Constants.java
 
b/pinot-controller/src/main/java/org/apache/pinot/controller/api/resources/Constants.java
index 3e9b33c14b..714fbc4ff0 100644
--- 
a/pinot-controller/src/main/java/org/apache/pinot/controller/api/resources/Constants.java
+++ 
b/pinot-controller/src/main/java/org/apache/pinot/controller/api/resources/Constants.java
@@ -46,6 +46,7 @@ public class Constants {
   public static final String ZOOKEEPER = "Zookeeper";
   public static final String APP_CONFIGS = "AppConfigs";
   public static final String PERIODIC_TASK_TAG = "PeriodicTask";
+  public static final String UPSERT_RESOURCE_TAG = "Upsert";
 
   public static TableType validateTableType(String tableTypeStr) {
     if (tableTypeStr == null || tableTypeStr.isEmpty()) {
diff --git 
a/pinot-controller/src/main/java/org/apache/pinot/controller/api/resources/PinotUpsertRestletResource.java
 
b/pinot-controller/src/main/java/org/apache/pinot/controller/api/resources/PinotUpsertRestletResource.java
new file mode 100644
index 0000000000..fb8227a090
--- /dev/null
+++ 
b/pinot-controller/src/main/java/org/apache/pinot/controller/api/resources/PinotUpsertRestletResource.java
@@ -0,0 +1,153 @@
+/**
+ * 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.controller.api.resources;
+
+import com.fasterxml.jackson.databind.node.ObjectNode;
+import io.swagger.annotations.Api;
+import io.swagger.annotations.ApiOperation;
+import io.swagger.annotations.ApiParam;
+import java.io.IOException;
+import java.util.List;
+import javax.ws.rs.Consumes;
+import javax.ws.rs.POST;
+import javax.ws.rs.Path;
+import javax.ws.rs.Produces;
+import javax.ws.rs.QueryParam;
+import javax.ws.rs.core.MediaType;
+import javax.ws.rs.core.Response;
+import 
org.apache.pinot.controller.api.exception.ControllerApplicationException;
+import org.apache.pinot.spi.config.table.TableConfig;
+import org.apache.pinot.spi.data.FieldSpec;
+import org.apache.pinot.spi.data.Schema;
+import org.apache.pinot.spi.utils.JsonUtils;
+import org.slf4j.Logger;
+import org.slf4j.LoggerFactory;
+
+
+@Api(tags = Constants.UPSERT_RESOURCE_TAG)
+@Path("/")
+public class PinotUpsertRestletResource {
+
+  public static final Logger LOGGER = 
LoggerFactory.getLogger(PinotUpsertRestletResource.class);
+
+  /**
+   * The API to estimate heap usage for a Pinot upsert table.
+   *
+   * Sample usage: provide tableConfig, tableSchema, and ColumnStats payload.
+   *
+   * The tool calculates heap usage by estimating total Key/Value space based 
on unique key combinations.
+   * It used the following formula
+   * ```
+   * TotalHeapSize = uniqueCombinations * (BytesPerKey + BytesPerValue).
+   * ```
+   * The following params need to be provided:
+   * ```
+   * -schemaFile, it contains primary key information.
+   * -tableConfigFile, it contains upsertConfig, tablePartitionConfig etc.
+   * -columnStats, which stores column information, collected from kafka or 
staging pinot table.
+   * ```
+   * For columns stats, we need to gather the following stats
+   * ```
+   * -cardinality, a required information unique combination of primary keys.
+   * -primaryKeySize, it uses for calculating BytesPerKey.
+   * -comparisonColSize, it uses for calculating BytesPerValue.
+   * -partitionNums(optional), it uses for host assignment calculation.
+   * ```
+   */
+  @POST
+  @Path("/upsert/estimateHeapUsage")
+  @Produces(MediaType.APPLICATION_JSON)
+  @Consumes(MediaType.APPLICATION_JSON)
+  @ApiOperation(value = "Estimate memory usage for an upsert table", notes =
+      "This API returns the estimated heap usage based on primary key column 
stats."
+          + " This allows us to estimate table size before onboarding.")
+  public String estimateHeapUsage(String tableSchemaConfigStr,
+      @ApiParam(value = "cardinality", required = true) 
@QueryParam("cardinality") long cardinality,
+      @ApiParam(value = "primaryKeySize", defaultValue = "-1") 
@QueryParam("primaryKeySize") int primaryKeySize,
+      @ApiParam(value = "numPartitions", defaultValue = "-1") 
@QueryParam("numPartitions") int numPartitions) {
+    ObjectNode resultData = JsonUtils.newObjectNode();
+    TableAndSchemaConfig tableSchemaConfig;
+
+    try {
+      tableSchemaConfig = JsonUtils.stringToObject(tableSchemaConfigStr, 
TableAndSchemaConfig.class);
+    } catch (IOException e) {
+      throw new ControllerApplicationException(LOGGER,
+          String.format("Invalid TableSchemaConfigs json string: %s", 
tableSchemaConfigStr),
+          Response.Status.BAD_REQUEST, e);
+    }
+
+    TableConfig tableConfig = tableSchemaConfig.getTableConfig();
+    resultData.put("tableName", tableConfig.getTableName());
+
+    Schema schema = tableSchemaConfig.getSchema();
+
+    // Estimated key space, it contains primary key columns.
+    int bytesPerKey = 0;
+    List<String> primaryKeys = schema.getPrimaryKeyColumns();
+
+    if (primaryKeySize > 0) {
+      bytesPerKey = primaryKeySize;
+    } else {
+      for (String primaryKey : primaryKeys) {
+        FieldSpec.DataType dt = 
schema.getFieldSpecFor(primaryKey).getDataType();
+        if (!dt.isFixedWidth()) {
+          String msg = "Primary key sizes much be provided for non fixed-width 
columns";
+          throw new ControllerApplicationException(LOGGER, msg, 
Response.Status.BAD_REQUEST);
+        } else {
+          bytesPerKey += dt.size();
+        }
+      }
+      // Java has a 24 bytes array overhead and there's also 8 bytes for the 
actual array object
+      bytesPerKey += 32;
+    }
+
+    // Estimated value space, it contains <segmentName, DocId, 
ComparisonValue(timestamp)> and overhead.
+    // Here we only calculate the map content size. TODO: Add the map entry 
size and the array size within the map.
+    int bytesPerValue = 60;
+    String comparisonColumn = 
tableConfig.getUpsertConfig().getComparisonColumn();
+    if (comparisonColumn != null) {
+      FieldSpec.DataType dt = 
schema.getFieldSpecFor(comparisonColumn).getDataType();
+      if (!dt.isFixedWidth()) {
+        String msg = "Not support data types for the comparison column";
+        throw new ControllerApplicationException(LOGGER, msg, 
Response.Status.BAD_REQUEST);
+      } else {
+        bytesPerValue = 52 + dt.size();
+      }
+    }
+
+    resultData.put("bytesPerKey", bytesPerKey);
+    resultData.put("bytesPerValue", bytesPerValue);
+
+    long totalKeySpace = bytesPerKey * cardinality;
+    long totalValueSpace = bytesPerValue * cardinality;
+    long totalSpace = totalKeySpace + totalValueSpace;
+
+    resultData.put("totalKeySpace(bytes)", totalKeySpace);
+    resultData.put("totalValueSpace(bytes)", totalValueSpace);
+    resultData.put("totalSpace(bytes)", totalSpace);
+
+    // Use Partitions, replicas to calculate memoryPerHost for host assignment.
+    if (numPartitions > 0) {
+      double totalSpacePerPartition = (totalSpace * 1.0) / numPartitions;
+      resultData.put("numPartitions", numPartitions);
+      resultData.put("totalSpacePerPartition(bytes)", totalSpacePerPartition);
+    }
+    return resultData.toString();
+  }
+}
diff --git 
a/pinot-controller/src/main/java/org/apache/pinot/controller/helix/ControllerRequestClient.java
 
b/pinot-controller/src/main/java/org/apache/pinot/controller/helix/ControllerRequestClient.java
index f189d4c2d7..bb6002903b 100644
--- 
a/pinot-controller/src/main/java/org/apache/pinot/controller/helix/ControllerRequestClient.java
+++ 
b/pinot-controller/src/main/java/org/apache/pinot/controller/helix/ControllerRequestClient.java
@@ -99,7 +99,7 @@ public class ControllerRequestClient {
       throws IOException {
     try {
       HttpClient.wrapAndThrowHttpException(_httpClient.sendJsonPutRequest(new 
URL(
-          
_controllerRequestURLBuilder.forUpdateTableConfig(tableConfig.getTableName())).toURI(),
+              
_controllerRequestURLBuilder.forUpdateTableConfig(tableConfig.getTableName())).toURI(),
           tableConfig.toJsonString()));
     } catch (HttpErrorStatusException | URISyntaxException e) {
       throw new IOException(e);
@@ -173,7 +173,7 @@ public class ControllerRequestClient {
       throws IOException {
     try {
       HttpClient.wrapAndThrowHttpException(_httpClient.sendJsonPostRequest(new 
URL(
-          _controllerRequestURLBuilder.forTenantCreate()).toURI(),
+              _controllerRequestURLBuilder.forTenantCreate()).toURI(),
           getBrokerTenantRequestPayload(tenantName, numBrokers)));
     } catch (HttpErrorStatusException | URISyntaxException e) {
       throw new IOException(e);
@@ -184,7 +184,7 @@ public class ControllerRequestClient {
       throws IOException {
     try {
       HttpClient.wrapAndThrowHttpException(_httpClient.sendJsonPutRequest(new 
URL(
-          _controllerRequestURLBuilder.forTenantCreate()).toURI(),
+              _controllerRequestURLBuilder.forTenantCreate()).toURI(),
           getBrokerTenantRequestPayload(tenantName, numBrokers)));
     } catch (HttpErrorStatusException | URISyntaxException e) {
       throw new IOException(e);
@@ -195,7 +195,7 @@ public class ControllerRequestClient {
       throws IOException {
     try {
       HttpClient.wrapAndThrowHttpException(_httpClient.sendJsonPostRequest(new 
URL(
-          _controllerRequestURLBuilder.forTenantCreate()).toURI(),
+              _controllerRequestURLBuilder.forTenantCreate()).toURI(),
           getServerTenantRequestPayload(tenantName, numOfflineServers, 
numRealtimeServers)));
     } catch (HttpErrorStatusException | URISyntaxException e) {
       throw new IOException(e);
@@ -206,7 +206,7 @@ public class ControllerRequestClient {
       throws IOException {
     try {
       HttpClient.wrapAndThrowHttpException(_httpClient.sendJsonPutRequest(new 
URL(
-          _controllerRequestURLBuilder.forTenantCreate()).toURI(),
+              _controllerRequestURLBuilder.forTenantCreate()).toURI(),
           getServerTenantRequestPayload(tenantName, numOfflineServers, 
numRealtimeServers)));
     } catch (HttpErrorStatusException | URISyntaxException e) {
       throw new IOException(e);
diff --git 
a/pinot-controller/src/test/java/org/apache/pinot/controller/ControllerTestUtils.java
 
b/pinot-controller/src/test/java/org/apache/pinot/controller/ControllerTestUtils.java
index 396321c1a4..b104cd6ae7 100644
--- 
a/pinot-controller/src/test/java/org/apache/pinot/controller/ControllerTestUtils.java
+++ 
b/pinot-controller/src/test/java/org/apache/pinot/controller/ControllerTestUtils.java
@@ -471,6 +471,12 @@ public abstract class ControllerTestUtils {
     return schema;
   }
 
+  public static Schema createDummySchemaForUpsertTable(String tableName) {
+    Schema schema = createDummySchema(tableName);
+    schema.setPrimaryKeyColumns(Collections.singletonList("dimA"));
+    return schema;
+  }
+
   public static void addDummySchema(String tableName)
       throws IOException {
     addSchema(createDummySchema(tableName));
diff --git 
a/pinot-controller/src/test/java/org/apache/pinot/controller/api/PinotUpsertRestletResourceTest.java
 
b/pinot-controller/src/test/java/org/apache/pinot/controller/api/PinotUpsertRestletResourceTest.java
new file mode 100644
index 0000000000..e68fc3c8e7
--- /dev/null
+++ 
b/pinot-controller/src/test/java/org/apache/pinot/controller/api/PinotUpsertRestletResourceTest.java
@@ -0,0 +1,78 @@
+/**
+ * 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.controller.api;
+
+import com.fasterxml.jackson.databind.JsonNode;
+import java.io.File;
+import java.net.URL;
+import org.apache.pinot.controller.ControllerTestUtils;
+import org.apache.pinot.controller.api.resources.TableAndSchemaConfig;
+import org.apache.pinot.spi.config.table.TableConfig;
+import org.apache.pinot.spi.data.Schema;
+import org.apache.pinot.spi.utils.JsonUtils;
+import org.testng.annotations.AfterClass;
+import org.testng.annotations.BeforeClass;
+import org.testng.annotations.Test;
+
+import static org.testng.Assert.assertEquals;
+
+
+public class PinotUpsertRestletResourceTest {
+  @BeforeClass
+  public void setUp()
+      throws Exception {
+    ControllerTestUtils.setupClusterAndValidate();
+  }
+
+  @Test
+  public void testEstimateHeapUsage()
+      throws Exception {
+
+    File schemaFile = readFile("memory_estimation/schema-for-upsert.json");
+    File tableConfigFile = 
readFile("memory_estimation/table-config-for-upsert.json");
+    Schema schema = JsonUtils.fileToObject(schemaFile, Schema.class);
+    TableConfig tableConfig = JsonUtils.fileToObject(tableConfigFile, 
TableConfig.class);
+
+    TableAndSchemaConfig tableAndSchemaConfig = new 
TableAndSchemaConfig(tableConfig, schema);
+
+    String estimateHeapUsageUrl =
+        
ControllerTestUtils.getControllerRequestURLBuilder().forUpsertTableHeapEstimation(10000,
 48, 8);
+
+    JsonNode result = JsonUtils.stringToJsonNode(
+        ControllerTestUtils.sendPostRequest(estimateHeapUsageUrl, 
tableAndSchemaConfig.toJsonString()));
+    assertEquals(result.get("bytesPerKey").asInt(), 48);
+    assertEquals(result.get("bytesPerValue").asInt(), 60);
+    assertEquals(result.get("totalKeySpace(bytes)").asLong(), 480000);
+    assertEquals(result.get("totalValueSpace(bytes)").asLong(), 600000);
+    assertEquals(result.get("totalSpace(bytes)").asLong(), 1080000);
+    assertEquals(result.get("numPartitions").asInt(), 8);
+    assertEquals(result.get("totalSpacePerPartition(bytes)").asDouble(), 
135000.0);
+  }
+
+  @AfterClass
+  public void tearDown() {
+    ControllerTestUtils.cleanup();
+  }
+
+  private File readFile(String fileName)
+      throws Exception {
+    URL resource = getClass().getClassLoader().getResource(fileName);
+    return new File(resource.toURI());
+  }
+}
diff --git 
a/pinot-controller/src/test/resources/memory_estimation/schema-for-upsert.json 
b/pinot-controller/src/test/resources/memory_estimation/schema-for-upsert.json
new file mode 100644
index 0000000000..7af367bca1
--- /dev/null
+++ 
b/pinot-controller/src/test/resources/memory_estimation/schema-for-upsert.json
@@ -0,0 +1,96 @@
+{
+  "schemaName": "restletTable_UPSERT",
+  "dimensionFieldSpecs": [
+    {
+      "dataType": "INT",
+      "name": "colInt",
+      "cardinality": 100
+    },
+    {
+      "dataType": "INT",
+      "name": "colIntMV",
+      "singleValueField": false,
+      "cardinality": 150,
+      "numValuesPerEntry": 3
+    },
+    {
+      "dataType": "FLOAT",
+      "name": "colFloat",
+      "cardinality": 200
+    },
+    {
+      "dataType": "FLOAT",
+      "name": "colFloatMV",
+      "singleValueField": false,
+      "cardinality": 250,
+      "numValuesPerEntry": 1.7
+    },
+    {
+      "dataType": "STRING",
+      "name": "colString",
+      "cardinality": 300,
+      "averageLength": 10
+    },
+    {
+      "dataType": "STRING",
+      "name": "colStringMV",
+      "singleValueField": false,
+      "cardinality": 350,
+      "averageLength": 10,
+      "numValuesPerEntry": 1.3
+    },
+    {
+      "dataType": "BYTES",
+      "name": "colBytes",
+      "cardinality": 400,
+      "averageLength": 5
+    },
+    {
+      "dataType": "LONG",
+      "name": "colLong",
+      "cardinality": 500
+    },
+    {
+      "dataType": "LONG",
+      "name": "colLongMV",
+      "singleValueField": false,
+      "cardinality": 550,
+      "numValuesPerEntry": 2.8
+    },
+    {
+      "dataType": "DOUBLE",
+      "name": "colDouble",
+      "cardinality": 600
+    },
+    {
+      "dataType": "DOUBLE",
+      "name": "colDoubleMV",
+      "singleValueField": false,
+      "cardinality": 650,
+      "numValuesPerEntry": 3.4
+    }
+  ],
+  "metricFieldSpecs": [
+    {
+      "dataType": "DOUBLE",
+      "name": "colDoubleMetric",
+      "cardinality": 700
+    },
+    {
+      "dataType": "FLOAT",
+      "name": "colFloatMetric",
+      "cardinality": 800
+    }
+  ],
+  "timeFieldSpec": {
+    "incomingGranularitySpec" : {
+      "dataType": "LONG",
+      "name": "colTime",
+      "timeType": "DAYS",
+      "cardinality": 900
+    }
+  },
+  "primaryKeyColumns": [
+    "colString"
+  ]
+}
diff --git 
a/pinot-controller/src/test/resources/memory_estimation/table-config-for-upsert.json
 
b/pinot-controller/src/test/resources/memory_estimation/table-config-for-upsert.json
new file mode 100644
index 0000000000..663f20bb36
--- /dev/null
+++ 
b/pinot-controller/src/test/resources/memory_estimation/table-config-for-upsert.json
@@ -0,0 +1,53 @@
+{
+  "metadata": {},
+  "routing": {
+    "routingTableBuilderName": "PartitionAwareRealtime",
+    "routingTableBuilderOptions": {},
+    "instanceSelectorType": "strictReplicaGroup"
+  },
+  "upsertConfig": {
+    "mode": "FULL"
+  },
+  "segmentsConfig": {
+    "replicasPerPartition": 3,
+    "replication": "3",
+    "retentionTimeUnit": "DAYS",
+    "retentionTimeValue": "5",
+    "schemaName": "restletTable_UPSERT",
+    "segmentAssignmentStrategy": "BalanceNumSegmentAssignmentStrategy",
+    "segmentPushFrequency": "daily",
+    "segmentPushType": "APPEND",
+    "timeColumnName": "colTime",
+    "timeType": "DAYS"
+  },
+  "tableIndexConfig": {
+    "aggregateMetrics": true,
+    "invertedIndexColumns": [
+      "colInt",
+      "colString"
+    ],
+    "loadMode": "MMAP",
+    "noDictionaryColumns": [
+      "colBytes"
+    ],
+    "segmentFormatVersion": "v3",
+    "sortedColumn": [],
+    "streamConfigs": {
+      "realtime.segment.flush.threshold.size": 100000000,
+      "realtime.segment.flush.threshold.time": "6h",
+      "stream.kafka.clusterGroup": "aggregate-tracking",
+      "stream.kafka.consumer.factory.class.name": 
"com.linkedin.pinot.v2.server.LiPinotKafkaConsumerFactory",
+      "stream.kafka.consumer.prop.auto.offset.reset": "largest",
+      "stream.kafka.consumer.type": "simple",
+      "stream.kafka.decoder.class.name": 
"com.linkedin.pinot.v2.server.LiKafkaDecoder",
+      "stream.kafka.topic.name": "UserGeneratedContentGestureCountEvent",
+      "streamType": "kafka"
+    }
+  },
+  "tableName": "restletTable_UPSERT",
+  "tableType": "REALTIME",
+  "tenants": {
+    "broker": "test",
+    "server": "test"
+  }
+}
diff --git 
a/pinot-spi/src/main/java/org/apache/pinot/spi/config/table/ColumnStats.java 
b/pinot-spi/src/main/java/org/apache/pinot/spi/config/table/ColumnStats.java
new file mode 100644
index 0000000000..c106d153e0
--- /dev/null
+++ b/pinot-spi/src/main/java/org/apache/pinot/spi/config/table/ColumnStats.java
@@ -0,0 +1,89 @@
+/**
+ * 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.spi.config.table;
+
+import com.fasterxml.jackson.annotation.JsonCreator;
+import com.fasterxml.jackson.annotation.JsonProperty;
+import com.fasterxml.jackson.core.JsonProcessingException;
+import javax.annotation.Nullable;
+import org.apache.pinot.spi.utils.JsonUtils;
+
+
+/*
+ * Container object for stats of Pinot columns for capacity estimation, output 
by kafka sampler.
+ */
+public class ColumnStats {
+
+  public static final String CARDINALITY = "cardinality";
+  public static final String PRIMARY_KEY_SIZE = "primaryKeySize";
+  public static final String NUM_PARTITIONS = "numPartitions";
+
+  private long _cardinality;
+  private int _primaryKeySize = 8;
+  private int _numPartitions = 0;
+
+  public ColumnStats(long cardinality) {
+    _cardinality = cardinality;
+  }
+
+  @JsonCreator
+  public ColumnStats(@JsonProperty(value = CARDINALITY, required = true) long 
cardinality,
+      @JsonProperty(value = PRIMARY_KEY_SIZE) int primaryKeySize,
+      @JsonProperty(value = NUM_PARTITIONS) int numPartitions) {
+    _cardinality = cardinality;
+    _primaryKeySize = primaryKeySize;
+    _numPartitions = numPartitions;
+  }
+
+  @JsonProperty(CARDINALITY)
+  public long getCardinality() {
+    return _cardinality;
+  }
+
+  @Nullable
+  @JsonProperty(PRIMARY_KEY_SIZE)
+  public int getPrimaryKeySize() {
+    return _primaryKeySize;
+  }
+
+  @JsonProperty(NUM_PARTITIONS)
+  public int getNumPartitions() {
+    return _numPartitions;
+  }
+
+  public void setCardinality(long cardinality) {
+    _cardinality = cardinality;
+  }
+
+  public void setPrimaryKeySize(int primaryKeySize) {
+    _primaryKeySize = primaryKeySize;
+  }
+
+  public void setNumPartitions(int numPartitions) {
+    _numPartitions = numPartitions;
+  }
+
+  public String toJsonString() {
+    try {
+      return JsonUtils.objectToString(this);
+    } catch (JsonProcessingException e) {
+      throw new RuntimeException(e);
+    }
+  }
+}
diff --git 
a/pinot-spi/src/main/java/org/apache/pinot/spi/utils/builder/ControllerRequestURLBuilder.java
 
b/pinot-spi/src/main/java/org/apache/pinot/spi/utils/builder/ControllerRequestURLBuilder.java
index 9f57222e2b..2cf765ff6b 100644
--- 
a/pinot-spi/src/main/java/org/apache/pinot/spi/utils/builder/ControllerRequestURLBuilder.java
+++ 
b/pinot-spi/src/main/java/org/apache/pinot/spi/utils/builder/ControllerRequestURLBuilder.java
@@ -425,6 +425,11 @@ public class ControllerRequestURLBuilder {
     return StringUtil.join("/", _baseUrl, "zk/getChildren", "?path=" + path);
   }
 
+  public String forUpsertTableHeapEstimation(long cardinality, int 
primaryKeySize, int numPartitions) {
+    return StringUtil.join("/", _baseUrl, "upsert/estimateHeapUsage",
+        "?cardinality=" + cardinality + "&primaryKeySize=" + primaryKeySize + 
"&numPartitions=" + numPartitions);
+  }
+
   private static String encode(String s) {
     try {
       return URLEncoder.encode(s, "UTF-8");


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

Reply via email to