You are viewing a plain text version of this content. The canonical link for it is here.
Posted to gitbox@hive.apache.org by "wecharyu (via GitHub)" <gi...@apache.org> on 2023/03/19 09:49:55 UTC

[GitHub] [hive] wecharyu opened a new pull request, #4123: HIVE-27150: Drop single partition can also support direct sql

wecharyu opened a new pull request, #4123:
URL: https://github.com/apache/hive/pull/4123

   ### What changes were proposed in this pull request?
   We want to reuse the `dropPartitionsInternal()` method for `drop_partition_common` api to enjoy the high performance, to achieve this we will refactor the `RawStore#dropPartition` api.
   
   
   ### Why are the changes needed?
   1. support direct sql for drop single partition
   2. reduce one query to the DB, that is MPartition query in `ObjectStore#dropPartition`
   
   
   ### Does this PR introduce _any_ user-facing change?
   No
   
   
   ### How was this patch tested?
   Pass all existing tests.
   


-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: gitbox-unsubscribe@hive.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


---------------------------------------------------------------------
To unsubscribe, e-mail: gitbox-unsubscribe@hive.apache.org
For additional commands, e-mail: gitbox-help@hive.apache.org


[GitHub] [hive] wecharyu commented on pull request #4123: HIVE-27150: Drop single partition can also support direct sql

Posted by "wecharyu (via GitHub)" <gi...@apache.org>.
wecharyu commented on PR #4123:
URL: https://github.com/apache/hive/pull/4123#issuecomment-1483018000

   @deniskuzZ @kasakrisz @saihemanth-cloudera: Could you please review this PR?


-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: gitbox-unsubscribe@hive.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


---------------------------------------------------------------------
To unsubscribe, e-mail: gitbox-unsubscribe@hive.apache.org
For additional commands, e-mail: gitbox-help@hive.apache.org


[GitHub] [hive] sonarcloud[bot] commented on pull request #4123: HIVE-27150: Drop single partition can also support direct sql

Posted by "sonarcloud[bot] (via GitHub)" <gi...@apache.org>.
sonarcloud[bot] commented on PR #4123:
URL: https://github.com/apache/hive/pull/4123#issuecomment-1475294531

   Kudos, SonarCloud Quality Gate passed!&nbsp; &nbsp; [![Quality Gate passed](https://sonarsource.github.io/sonarcloud-github-static-resources/v2/checks/QualityGateBadge/passed-16px.png 'Quality Gate passed')](https://sonarcloud.io/dashboard?id=apache_hive&pullRequest=4123)
   
   [![Bug](https://sonarsource.github.io/sonarcloud-github-static-resources/v2/common/bug-16px.png 'Bug')](https://sonarcloud.io/project/issues?id=apache_hive&pullRequest=4123&resolved=false&types=BUG) [![A](https://sonarsource.github.io/sonarcloud-github-static-resources/v2/checks/RatingBadge/A-16px.png 'A')](https://sonarcloud.io/project/issues?id=apache_hive&pullRequest=4123&resolved=false&types=BUG) [0 Bugs](https://sonarcloud.io/project/issues?id=apache_hive&pullRequest=4123&resolved=false&types=BUG)  
   [![Vulnerability](https://sonarsource.github.io/sonarcloud-github-static-resources/v2/common/vulnerability-16px.png 'Vulnerability')](https://sonarcloud.io/project/issues?id=apache_hive&pullRequest=4123&resolved=false&types=VULNERABILITY) [![A](https://sonarsource.github.io/sonarcloud-github-static-resources/v2/checks/RatingBadge/A-16px.png 'A')](https://sonarcloud.io/project/issues?id=apache_hive&pullRequest=4123&resolved=false&types=VULNERABILITY) [0 Vulnerabilities](https://sonarcloud.io/project/issues?id=apache_hive&pullRequest=4123&resolved=false&types=VULNERABILITY)  
   [![Security Hotspot](https://sonarsource.github.io/sonarcloud-github-static-resources/v2/common/security_hotspot-16px.png 'Security Hotspot')](https://sonarcloud.io/project/security_hotspots?id=apache_hive&pullRequest=4123&resolved=false&types=SECURITY_HOTSPOT) [![A](https://sonarsource.github.io/sonarcloud-github-static-resources/v2/checks/RatingBadge/A-16px.png 'A')](https://sonarcloud.io/project/security_hotspots?id=apache_hive&pullRequest=4123&resolved=false&types=SECURITY_HOTSPOT) [0 Security Hotspots](https://sonarcloud.io/project/security_hotspots?id=apache_hive&pullRequest=4123&resolved=false&types=SECURITY_HOTSPOT)  
   [![Code Smell](https://sonarsource.github.io/sonarcloud-github-static-resources/v2/common/code_smell-16px.png 'Code Smell')](https://sonarcloud.io/project/issues?id=apache_hive&pullRequest=4123&resolved=false&types=CODE_SMELL) [![A](https://sonarsource.github.io/sonarcloud-github-static-resources/v2/checks/RatingBadge/A-16px.png 'A')](https://sonarcloud.io/project/issues?id=apache_hive&pullRequest=4123&resolved=false&types=CODE_SMELL) [6 Code Smells](https://sonarcloud.io/project/issues?id=apache_hive&pullRequest=4123&resolved=false&types=CODE_SMELL)
   
   [![No Coverage information](https://sonarsource.github.io/sonarcloud-github-static-resources/v2/checks/CoverageChart/NoCoverageInfo-16px.png 'No Coverage information')](https://sonarcloud.io/component_measures?id=apache_hive&pullRequest=4123&metric=coverage&view=list) No Coverage information  
   [![No Duplication information](https://sonarsource.github.io/sonarcloud-github-static-resources/v2/checks/Duplications/NoDuplicationInfo-16px.png 'No Duplication information')](https://sonarcloud.io/component_measures?id=apache_hive&pullRequest=4123&metric=duplicated_lines_density&view=list) No Duplication information
   
   


-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: gitbox-unsubscribe@hive.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


---------------------------------------------------------------------
To unsubscribe, e-mail: gitbox-unsubscribe@hive.apache.org
For additional commands, e-mail: gitbox-help@hive.apache.org


[GitHub] [hive] VenuReddy2103 commented on a diff in pull request #4123: HIVE-27150: Drop single partition can also support direct sql

Posted by "VenuReddy2103 (via GitHub)" <gi...@apache.org>.
VenuReddy2103 commented on code in PR #4123:
URL: https://github.com/apache/hive/pull/4123#discussion_r1150463962


##########
standalone-metastore/metastore-server/src/main/java/org/apache/hadoop/hive/metastore/RawStore.java:
##########
@@ -459,16 +459,15 @@ boolean doesPartitionExist(String catName, String dbName, String tableName,
    * @param catName catalog name.
    * @param dbName database name.
    * @param tableName table name.
-   * @param part_vals list of partition values.
+   * @param partName partition name.
    * @return true if the partition was dropped.
    * @throws MetaException Error accessing the RDBMS.
    * @throws NoSuchObjectException no partition matching this description exists
    * @throws InvalidObjectException error dropping the statistics for the partition
    * @throws InvalidInputException error dropping the statistics for the partition
    */
-  boolean dropPartition(String catName, String dbName, String tableName,
-      List<String> part_vals) throws MetaException, NoSuchObjectException, InvalidObjectException,
-      InvalidInputException;
+  boolean dropPartition(String catName, String dbName, String tableName, String partName)

Review Comment:
   IMHO, Instead of defining this new API, we can make the partname inside the existing `dropPartition()` method itself to invoke `dropPartitionsInternal()` or `dropPartitions()`. Because the new API signature is similar to exisiting `dropPartitions()` except the last argument(i.e., single partname vs list of partnames). May be, we can mark `dropPartition()` as deprecated and insist using `dropPartitions()` directly in future.



-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: gitbox-unsubscribe@hive.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


---------------------------------------------------------------------
To unsubscribe, e-mail: gitbox-unsubscribe@hive.apache.org
For additional commands, e-mail: gitbox-help@hive.apache.org


[GitHub] [hive] saihemanth-cloudera commented on a diff in pull request #4123: HIVE-27150: Drop single partition can also support direct sql

Posted by "saihemanth-cloudera (via GitHub)" <gi...@apache.org>.
saihemanth-cloudera commented on code in PR #4123:
URL: https://github.com/apache/hive/pull/4123#discussion_r1154298770


##########
standalone-metastore/metastore-server/src/main/java/org/apache/hadoop/hive/metastore/HMSHandler.java:
##########
@@ -5026,7 +5026,8 @@ private boolean drop_partition_common(RawStore ms, String catName, String db_nam
         verifyIsWritablePath(partPath);
       }
 
-      if (!ms.dropPartition(catName, db_name, tbl_name, part_vals)) {
+      String partName = Warehouse.makePartName(tbl.getPartitionKeys(), part_vals);
+      if (!ms.dropPartition(catName, db_name, tbl_name, partName)) {
         throw new MetaException("Unable to drop partition");

Review Comment:
   Should we throw the exception stack trace/reason for failure?



-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: gitbox-unsubscribe@hive.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


---------------------------------------------------------------------
To unsubscribe, e-mail: gitbox-unsubscribe@hive.apache.org
For additional commands, e-mail: gitbox-help@hive.apache.org


[GitHub] [hive] wecharyu commented on a diff in pull request #4123: HIVE-27150: Drop single partition can also support direct sql

Posted by "wecharyu (via GitHub)" <gi...@apache.org>.
wecharyu commented on code in PR #4123:
URL: https://github.com/apache/hive/pull/4123#discussion_r1162961355


##########
standalone-metastore/metastore-server/src/test/java/org/apache/hadoop/hive/metastore/TestObjectStore.java:
##########
@@ -498,6 +497,68 @@ public void testPartitionOpsWhenTableDoesNotExist() throws InvalidObjectExceptio
     }
   }
 
+  @Test
+  public void testDropPartitionByName() throws Exception {
+    Database db1 = new DatabaseBuilder()
+        .setName(DB1)
+        .setDescription("description")
+        .setLocation("locationurl")
+        .build(conf);
+    try (AutoCloseable c = deadline()) {
+      objectStore.createDatabase(db1);
+    }
+    StorageDescriptor sd = createFakeSd("location");
+    HashMap<String, String> tableParams = new HashMap<>();
+    tableParams.put("EXTERNAL", "false");
+    FieldSchema partitionKey1 = new FieldSchema("Country", ColumnType.STRING_TYPE_NAME, "");
+    FieldSchema partitionKey2 = new FieldSchema("State", ColumnType.STRING_TYPE_NAME, "");
+    Table tbl1 =
+        new Table(TABLE1, DB1, "owner", 1, 2, 3, sd, Arrays.asList(partitionKey1, partitionKey2),
+            tableParams, null, null, "MANAGED_TABLE");
+    try (AutoCloseable c = deadline()) {
+      objectStore.createTable(tbl1);
+    }
+    HashMap<String, String> partitionParams = new HashMap<>();
+    partitionParams.put("PARTITION_LEVEL_PRIVILEGE", "true");
+    List<String> value1 = Arrays.asList("US", "CA");
+    Partition part1 = new Partition(value1, DB1, TABLE1, 111, 111, sd, partitionParams);
+    part1.setCatName(DEFAULT_CATALOG_NAME);
+    try (AutoCloseable c = deadline()) {
+      objectStore.addPartition(part1);
+    }
+    List<String> value2 = Arrays.asList("US", "MA");
+    Partition part2 = new Partition(value2, DB1, TABLE1, 222, 222, sd, partitionParams);
+    part2.setCatName(DEFAULT_CATALOG_NAME);
+    try (AutoCloseable c = deadline()) {
+      objectStore.addPartition(part2);
+    }
+
+    List<Partition> partitions;
+    try (AutoCloseable c = deadline()) {
+      objectStore.dropPartition(DEFAULT_CATALOG_NAME, DB1, TABLE1, "country=US/state=CA");
+      partitions = objectStore.getPartitions(DEFAULT_CATALOG_NAME, DB1, TABLE1, 10);
+    }
+    Assert.assertEquals(1, partitions.size());
+    Assert.assertEquals(222, partitions.get(0).getCreateTime());
+    try (AutoCloseable c = deadline()) {
+      objectStore.dropPartition(DEFAULT_CATALOG_NAME, DB1, TABLE1, "country=US/state=MA");
+      partitions = objectStore.getPartitions(DEFAULT_CATALOG_NAME, DB1, TABLE1, 10);
+    }
+    Assert.assertEquals(0, partitions.size());
+
+    try (AutoCloseable c = deadline()) {
+      // Illegal partName will do nothing, it doesn't matter
+      // because the real HMSHandler will guarantee the partName is legal and exists.
+      objectStore.dropPartition(DEFAULT_CATALOG_NAME, DB1, TABLE1, "country=US/state=NON_EXIST");
+      objectStore.dropPartition(DEFAULT_CATALOG_NAME, DB1, TABLE1, "country=US/st=CA");

Review Comment:
   It will not return false for non-existent partition, `HMSHandler` has already checked the existence of the partition and it does not need a double check in `ObjectStore`.



-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: gitbox-unsubscribe@hive.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


---------------------------------------------------------------------
To unsubscribe, e-mail: gitbox-unsubscribe@hive.apache.org
For additional commands, e-mail: gitbox-help@hive.apache.org


[GitHub] [hive] saihemanth-cloudera commented on a diff in pull request #4123: HIVE-27150: Drop single partition can also support direct sql

Posted by "saihemanth-cloudera (via GitHub)" <gi...@apache.org>.
saihemanth-cloudera commented on code in PR #4123:
URL: https://github.com/apache/hive/pull/4123#discussion_r1162143358


##########
standalone-metastore/metastore-server/src/main/java/org/apache/hadoop/hive/metastore/HMSHandler.java:
##########
@@ -5026,20 +5026,18 @@ private boolean drop_partition_common(RawStore ms, String catName, String db_nam
         verifyIsWritablePath(partPath);
       }
 
-      if (!ms.dropPartition(catName, db_name, tbl_name, part_vals)) {
-        throw new MetaException("Unable to drop partition");
-      } else {
-        if (!transactionalListeners.isEmpty()) {
+      String partName = Warehouse.makePartName(tbl.getPartitionKeys(), part_vals);
+      ms.dropPartition(catName, db_name, tbl_name, partName);

Review Comment:
   If dropPartition in the object store is not successful, then we should throw a meta exception, right? 



-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: gitbox-unsubscribe@hive.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


---------------------------------------------------------------------
To unsubscribe, e-mail: gitbox-unsubscribe@hive.apache.org
For additional commands, e-mail: gitbox-help@hive.apache.org


[GitHub] [hive] saihemanth-cloudera commented on a diff in pull request #4123: HIVE-27150: Drop single partition can also support direct sql

Posted by "saihemanth-cloudera (via GitHub)" <gi...@apache.org>.
saihemanth-cloudera commented on code in PR #4123:
URL: https://github.com/apache/hive/pull/4123#discussion_r1152712297


##########
standalone-metastore/metastore-server/src/main/java/org/apache/hadoop/hive/metastore/RawStore.java:
##########
@@ -459,16 +459,15 @@ boolean doesPartitionExist(String catName, String dbName, String tableName,
    * @param catName catalog name.
    * @param dbName database name.
    * @param tableName table name.
-   * @param part_vals list of partition values.
+   * @param partName partition name.
    * @return true if the partition was dropped.
    * @throws MetaException Error accessing the RDBMS.
    * @throws NoSuchObjectException no partition matching this description exists
    * @throws InvalidObjectException error dropping the statistics for the partition
    * @throws InvalidInputException error dropping the statistics for the partition
    */
-  boolean dropPartition(String catName, String dbName, String tableName,
-      List<String> part_vals) throws MetaException, NoSuchObjectException, InvalidObjectException,
-      InvalidInputException;
+  boolean dropPartition(String catName, String dbName, String tableName, String partName)

Review Comment:
   I would favor this change instead of passing the table into the drop partition API in the object store.



-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: gitbox-unsubscribe@hive.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


---------------------------------------------------------------------
To unsubscribe, e-mail: gitbox-unsubscribe@hive.apache.org
For additional commands, e-mail: gitbox-help@hive.apache.org


[GitHub] [hive] wecharyu commented on a diff in pull request #4123: HIVE-27150: Drop single partition can also support direct sql

Posted by "wecharyu (via GitHub)" <gi...@apache.org>.
wecharyu commented on code in PR #4123:
URL: https://github.com/apache/hive/pull/4123#discussion_r1154494889


##########
standalone-metastore/metastore-server/src/main/java/org/apache/hadoop/hive/metastore/HMSHandler.java:
##########
@@ -5026,7 +5026,8 @@ private boolean drop_partition_common(RawStore ms, String catName, String db_nam
         verifyIsWritablePath(partPath);
       }
 
-      if (!ms.dropPartition(catName, db_name, tbl_name, part_vals)) {
+      String partName = Warehouse.makePartName(tbl.getPartitionKeys(), part_vals);
+      if (!ms.dropPartition(catName, db_name, tbl_name, partName)) {
         throw new MetaException("Unable to drop partition");

Review Comment:
   `ms.dropPartition()` will throw the exception, we do not need to throw this `MetaException` here, will remove this code.



-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: gitbox-unsubscribe@hive.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


---------------------------------------------------------------------
To unsubscribe, e-mail: gitbox-unsubscribe@hive.apache.org
For additional commands, e-mail: gitbox-help@hive.apache.org


[GitHub] [hive] saihemanth-cloudera commented on a diff in pull request #4123: HIVE-27150: Drop single partition can also support direct sql

Posted by "saihemanth-cloudera (via GitHub)" <gi...@apache.org>.
saihemanth-cloudera commented on code in PR #4123:
URL: https://github.com/apache/hive/pull/4123#discussion_r1154338570


##########
standalone-metastore/metastore-tools/metastore-benchmarks/src/main/java/org/apache/hadoop/hive/metastore/tools/HMSBenchmarks.java:
##########
@@ -246,26 +246,24 @@ static DescriptiveStatistics benchmarkGetPartitions(@NotNull MicroBenchmark benc
   }
 
   static DescriptiveStatistics benchmarkDropPartition(@NotNull MicroBenchmark bench,
-                                                      @NotNull BenchData data) {
+                                                      @NotNull BenchData data,
+                                                      int count) {
     final HMSClient client = data.getClient();
     String dbName = data.dbName;
     String tableName = data.tableName;
 
     BenchmarkUtils.createPartitionedTable(client, dbName, tableName);
-    final List<String> values = Collections.singletonList("d1");
     try {
-      Table t = client.getTable(dbName, tableName);
-      Partition partition = new Util.PartitionBuilder(t)
-          .withValues(values)
-          .build();
-
       return bench.measure(
-          () -> throwingSupplierWrapper(() -> client.addPartition(partition)),
-          () -> throwingSupplierWrapper(() -> client.dropPartition(dbName, tableName, values)),
+          () -> addManyPartitionsNoException(client, dbName, tableName, null,
+                  Collections.singletonList("d"), count),
+          () -> throwingSupplierWrapper(() -> {
+            List<String> partNames = client.getPartitionNames(dbName, tableName);
+            partNames.forEach(partName ->
+                throwingSupplierWrapper(() -> client.dropPartition(dbName, tableName, partName)));
+            return null;
+          }),
           null);
-    } catch (TException e) {

Review Comment:
   Why do we need to remove this? I think we are somehow the percentage of error records is not correct. 



-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: gitbox-unsubscribe@hive.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


---------------------------------------------------------------------
To unsubscribe, e-mail: gitbox-unsubscribe@hive.apache.org
For additional commands, e-mail: gitbox-help@hive.apache.org


[GitHub] [hive] wecharyu commented on a diff in pull request #4123: HIVE-27150: Drop single partition can also support direct sql

Posted by "wecharyu (via GitHub)" <gi...@apache.org>.
wecharyu commented on code in PR #4123:
URL: https://github.com/apache/hive/pull/4123#discussion_r1152971682


##########
standalone-metastore/metastore-server/src/main/java/org/apache/hadoop/hive/metastore/RawStore.java:
##########
@@ -459,16 +459,15 @@ boolean doesPartitionExist(String catName, String dbName, String tableName,
    * @param catName catalog name.
    * @param dbName database name.
    * @param tableName table name.
-   * @param part_vals list of partition values.
+   * @param partName partition name.
    * @return true if the partition was dropped.
    * @throws MetaException Error accessing the RDBMS.
    * @throws NoSuchObjectException no partition matching this description exists
    * @throws InvalidObjectException error dropping the statistics for the partition
    * @throws InvalidInputException error dropping the statistics for the partition
    */
-  boolean dropPartition(String catName, String dbName, String tableName,
-      List<String> part_vals) throws MetaException, NoSuchObjectException, InvalidObjectException,
-      InvalidInputException;
+  boolean dropPartition(String catName, String dbName, String tableName, String partName)

Review Comment:
   Got it! Thank you.



-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: gitbox-unsubscribe@hive.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


---------------------------------------------------------------------
To unsubscribe, e-mail: gitbox-unsubscribe@hive.apache.org
For additional commands, e-mail: gitbox-help@hive.apache.org


[GitHub] [hive] saihemanth-cloudera commented on a diff in pull request #4123: HIVE-27150: Drop single partition can also support direct sql

Posted by "saihemanth-cloudera (via GitHub)" <gi...@apache.org>.
saihemanth-cloudera commented on code in PR #4123:
URL: https://github.com/apache/hive/pull/4123#discussion_r1148824344


##########
standalone-metastore/metastore-server/src/main/java/org/apache/hadoop/hive/metastore/ObjectStore.java:
##########
@@ -3101,6 +3100,22 @@ public boolean dropPartition(String catName, String dbName, String tableName,
     return success;
   }
 
+  @Override
+  public boolean dropPartition(String catName, String dbName, String tableName, String partName)
+      throws MetaException, NoSuchObjectException, InvalidObjectException, InvalidInputException {
+    boolean success = false;
+    try {
+      openTransaction();
+      dropPartitionsInternal(catName, dbName, tableName, Arrays.asList(partName), true, true);

Review Comment:
   I don't think this would improve the performance by any means. Consider dropping 10k partitions, each partition would have to access same number of tables in the underlying db to update the records, so it makes sense to batch them and implement with direct SQL. But for a single partition since it'll access same number of tables in the DB, I don't think it'll make sense to implement this feature.
   For example, [HIVE-26035](https://issues.apache.org/jira/browse/HIVE-26035) (see the details in the jira) proved that implementing direct SQL actually improved the performance by running against benchmark tests.
   Similarly can you provide any evidence that this patch also has an edge by running those tests? (Probably you might have to add some tests(e.g: dropping 10K+ single partitions).



-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: gitbox-unsubscribe@hive.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


---------------------------------------------------------------------
To unsubscribe, e-mail: gitbox-unsubscribe@hive.apache.org
For additional commands, e-mail: gitbox-help@hive.apache.org


[GitHub] [hive] saihemanth-cloudera commented on a diff in pull request #4123: HIVE-27150: Drop single partition can also support direct sql

Posted by "saihemanth-cloudera (via GitHub)" <gi...@apache.org>.
saihemanth-cloudera commented on code in PR #4123:
URL: https://github.com/apache/hive/pull/4123#discussion_r1150050461


##########
standalone-metastore/metastore-server/src/main/java/org/apache/hadoop/hive/metastore/ObjectStore.java:
##########
@@ -3101,6 +3100,22 @@ public boolean dropPartition(String catName, String dbName, String tableName,
     return success;
   }
 
+  @Override
+  public boolean dropPartition(String catName, String dbName, String tableName, String partName)
+      throws MetaException, NoSuchObjectException, InvalidObjectException, InvalidInputException {
+    boolean success = false;
+    try {
+      openTransaction();
+      dropPartitionsInternal(catName, dbName, tableName, Arrays.asList(partName), true, true);

Review Comment:
   cc @VenuReddy2103 



-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: gitbox-unsubscribe@hive.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


---------------------------------------------------------------------
To unsubscribe, e-mail: gitbox-unsubscribe@hive.apache.org
For additional commands, e-mail: gitbox-help@hive.apache.org


[GitHub] [hive] saihemanth-cloudera commented on a diff in pull request #4123: HIVE-27150: Drop single partition can also support direct sql

Posted by "saihemanth-cloudera (via GitHub)" <gi...@apache.org>.
saihemanth-cloudera commented on code in PR #4123:
URL: https://github.com/apache/hive/pull/4123#discussion_r1154326620


##########
standalone-metastore/metastore-server/src/test/java/org/apache/hadoop/hive/metastore/TestObjectStore.java:
##########
@@ -439,14 +439,14 @@ public void testPartitionOps() throws Exception {
     Assert.assertEquals(2, numPartitions);
 
     try (AutoCloseable c = deadline()) {
-      objectStore.dropPartition(DEFAULT_CATALOG_NAME, DB1, TABLE1, value1);

Review Comment:
   Ideally, I would like to keep the old tests and add new tests that takes in "partName" as argument.



-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: gitbox-unsubscribe@hive.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


---------------------------------------------------------------------
To unsubscribe, e-mail: gitbox-unsubscribe@hive.apache.org
For additional commands, e-mail: gitbox-help@hive.apache.org


[GitHub] [hive] wecharyu commented on pull request #4123: HIVE-27150: Drop single partition can also support direct sql

Posted by "wecharyu (via GitHub)" <gi...@apache.org>.
wecharyu commented on PR #4123:
URL: https://github.com/apache/hive/pull/4123#issuecomment-1492359832

   Address comments. cc: @saihemanth-cloudera 


-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: gitbox-unsubscribe@hive.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


---------------------------------------------------------------------
To unsubscribe, e-mail: gitbox-unsubscribe@hive.apache.org
For additional commands, e-mail: gitbox-help@hive.apache.org


[GitHub] [hive] wecharyu commented on a diff in pull request #4123: HIVE-27150: Drop single partition can also support direct sql

Posted by "wecharyu (via GitHub)" <gi...@apache.org>.
wecharyu commented on code in PR #4123:
URL: https://github.com/apache/hive/pull/4123#discussion_r1154570802


##########
standalone-metastore/metastore-tools/metastore-benchmarks/src/main/java/org/apache/hadoop/hive/metastore/tools/HMSBenchmarks.java:
##########
@@ -246,26 +246,24 @@ static DescriptiveStatistics benchmarkGetPartitions(@NotNull MicroBenchmark benc
   }
 
   static DescriptiveStatistics benchmarkDropPartition(@NotNull MicroBenchmark bench,
-                                                      @NotNull BenchData data) {
+                                                      @NotNull BenchData data,
+                                                      int count) {
     final HMSClient client = data.getClient();
     String dbName = data.dbName;
     String tableName = data.tableName;
 
     BenchmarkUtils.createPartitionedTable(client, dbName, tableName);
-    final List<String> values = Collections.singletonList("d1");
     try {
-      Table t = client.getTable(dbName, tableName);
-      Partition partition = new Util.PartitionBuilder(t)
-          .withValues(values)
-          .build();
-
       return bench.measure(
-          () -> throwingSupplierWrapper(() -> client.addPartition(partition)),
-          () -> throwingSupplierWrapper(() -> client.dropPartition(dbName, tableName, values)),
+          () -> addManyPartitionsNoException(client, dbName, tableName, null,
+                  Collections.singletonList("d"), count),
+          () -> throwingSupplierWrapper(() -> {
+            List<String> partNames = client.getPartitionNames(dbName, tableName);
+            partNames.forEach(partName ->
+                throwingSupplierWrapper(() -> client.dropPartition(dbName, tableName, partName)));
+            return null;
+          }),
           null);
-    } catch (TException e) {

Review Comment:
   I followed the `benchmarkDropPartitions()` and some others, catching exception here should have nothing to do with benchmark statistics, because the result statistic is in `measure()` method:
   https://github.com/apache/hive/blob/745dbf7059de93a74bceda5e5acbb293008771dc/standalone-metastore/metastore-tools/tools-common/src/main/java/org/apache/hadoop/hive/metastore/tools/MicroBenchmark.java#L84-L87
   
   Also the `Err%` should be **Coefficient of variation** of successful operations rather than percentage of error records:
   https://github.com/apache/hive/blob/745dbf7059de93a74bceda5e5acbb293008771dc/standalone-metastore/metastore-tools/tools-common/src/main/java/org/apache/hadoop/hive/metastore/tools/BenchmarkSuite.java#L207



-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: gitbox-unsubscribe@hive.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


---------------------------------------------------------------------
To unsubscribe, e-mail: gitbox-unsubscribe@hive.apache.org
For additional commands, e-mail: gitbox-help@hive.apache.org


[GitHub] [hive] wecharyu commented on a diff in pull request #4123: HIVE-27150: Drop single partition can also support direct sql

Posted by "wecharyu (via GitHub)" <gi...@apache.org>.
wecharyu commented on code in PR #4123:
URL: https://github.com/apache/hive/pull/4123#discussion_r1162948940


##########
standalone-metastore/metastore-server/src/main/java/org/apache/hadoop/hive/metastore/HMSHandler.java:
##########
@@ -5026,20 +5026,18 @@ private boolean drop_partition_common(RawStore ms, String catName, String db_nam
         verifyIsWritablePath(partPath);
       }
 
-      if (!ms.dropPartition(catName, db_name, tbl_name, part_vals)) {
-        throw new MetaException("Unable to drop partition");
-      } else {
-        if (!transactionalListeners.isEmpty()) {
+      String partName = Warehouse.makePartName(tbl.getPartitionKeys(), part_vals);
+      ms.dropPartition(catName, db_name, tbl_name, partName);

Review Comment:
   Yes, it will throw a meta exception by `GetHelper`:
   https://github.com/apache/hive/blob/e397acd81ce4a909c5a564008117067df73872a5/standalone-metastore/metastore-server/src/main/java/org/apache/hadoop/hive/metastore/ObjectStore.java#L4435-L4439



-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: gitbox-unsubscribe@hive.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


---------------------------------------------------------------------
To unsubscribe, e-mail: gitbox-unsubscribe@hive.apache.org
For additional commands, e-mail: gitbox-help@hive.apache.org


[GitHub] [hive] deniskuzZ commented on a diff in pull request #4123: HIVE-27150: Drop single partition can also support direct sql

Posted by "deniskuzZ (via GitHub)" <gi...@apache.org>.
deniskuzZ commented on code in PR #4123:
URL: https://github.com/apache/hive/pull/4123#discussion_r1148088843


##########
standalone-metastore/metastore-server/src/main/java/org/apache/hadoop/hive/metastore/RawStore.java:
##########
@@ -459,16 +459,15 @@ boolean doesPartitionExist(String catName, String dbName, String tableName,
    * @param catName catalog name.
    * @param dbName database name.
    * @param tableName table name.
-   * @param part_vals list of partition values.
+   * @param partName partition name.
    * @return true if the partition was dropped.
    * @throws MetaException Error accessing the RDBMS.
    * @throws NoSuchObjectException no partition matching this description exists
    * @throws InvalidObjectException error dropping the statistics for the partition
    * @throws InvalidInputException error dropping the statistics for the partition
    */
-  boolean dropPartition(String catName, String dbName, String tableName,
-      List<String> part_vals) throws MetaException, NoSuchObjectException, InvalidObjectException,
-      InvalidInputException;
+  boolean dropPartition(String catName, String dbName, String tableName, String partName)

Review Comment:
   That is a change in API that breaks backward compatibility. We should keep the original signature and mark it as deprecated. In the following releases we could remove it.



-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: gitbox-unsubscribe@hive.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


---------------------------------------------------------------------
To unsubscribe, e-mail: gitbox-unsubscribe@hive.apache.org
For additional commands, e-mail: gitbox-help@hive.apache.org


[GitHub] [hive] sonarcloud[bot] commented on pull request #4123: HIVE-27150: Drop single partition can also support direct sql

Posted by "sonarcloud[bot] (via GitHub)" <gi...@apache.org>.
sonarcloud[bot] commented on PR #4123:
URL: https://github.com/apache/hive/pull/4123#issuecomment-1483766759

   Kudos, SonarCloud Quality Gate passed!&nbsp; &nbsp; [![Quality Gate passed](https://sonarsource.github.io/sonarcloud-github-static-resources/v2/checks/QualityGateBadge/passed-16px.png 'Quality Gate passed')](https://sonarcloud.io/dashboard?id=apache_hive&pullRequest=4123)
   
   [![Bug](https://sonarsource.github.io/sonarcloud-github-static-resources/v2/common/bug-16px.png 'Bug')](https://sonarcloud.io/project/issues?id=apache_hive&pullRequest=4123&resolved=false&types=BUG) [![A](https://sonarsource.github.io/sonarcloud-github-static-resources/v2/checks/RatingBadge/A-16px.png 'A')](https://sonarcloud.io/project/issues?id=apache_hive&pullRequest=4123&resolved=false&types=BUG) [0 Bugs](https://sonarcloud.io/project/issues?id=apache_hive&pullRequest=4123&resolved=false&types=BUG)  
   [![Vulnerability](https://sonarsource.github.io/sonarcloud-github-static-resources/v2/common/vulnerability-16px.png 'Vulnerability')](https://sonarcloud.io/project/issues?id=apache_hive&pullRequest=4123&resolved=false&types=VULNERABILITY) [![A](https://sonarsource.github.io/sonarcloud-github-static-resources/v2/checks/RatingBadge/A-16px.png 'A')](https://sonarcloud.io/project/issues?id=apache_hive&pullRequest=4123&resolved=false&types=VULNERABILITY) [0 Vulnerabilities](https://sonarcloud.io/project/issues?id=apache_hive&pullRequest=4123&resolved=false&types=VULNERABILITY)  
   [![Security Hotspot](https://sonarsource.github.io/sonarcloud-github-static-resources/v2/common/security_hotspot-16px.png 'Security Hotspot')](https://sonarcloud.io/project/security_hotspots?id=apache_hive&pullRequest=4123&resolved=false&types=SECURITY_HOTSPOT) [![A](https://sonarsource.github.io/sonarcloud-github-static-resources/v2/checks/RatingBadge/A-16px.png 'A')](https://sonarcloud.io/project/security_hotspots?id=apache_hive&pullRequest=4123&resolved=false&types=SECURITY_HOTSPOT) [0 Security Hotspots](https://sonarcloud.io/project/security_hotspots?id=apache_hive&pullRequest=4123&resolved=false&types=SECURITY_HOTSPOT)  
   [![Code Smell](https://sonarsource.github.io/sonarcloud-github-static-resources/v2/common/code_smell-16px.png 'Code Smell')](https://sonarcloud.io/project/issues?id=apache_hive&pullRequest=4123&resolved=false&types=CODE_SMELL) [![A](https://sonarsource.github.io/sonarcloud-github-static-resources/v2/checks/RatingBadge/A-16px.png 'A')](https://sonarcloud.io/project/issues?id=apache_hive&pullRequest=4123&resolved=false&types=CODE_SMELL) [7 Code Smells](https://sonarcloud.io/project/issues?id=apache_hive&pullRequest=4123&resolved=false&types=CODE_SMELL)
   
   [![No Coverage information](https://sonarsource.github.io/sonarcloud-github-static-resources/v2/checks/CoverageChart/NoCoverageInfo-16px.png 'No Coverage information')](https://sonarcloud.io/component_measures?id=apache_hive&pullRequest=4123&metric=coverage&view=list) No Coverage information  
   [![No Duplication information](https://sonarsource.github.io/sonarcloud-github-static-resources/v2/checks/Duplications/NoDuplicationInfo-16px.png 'No Duplication information')](https://sonarcloud.io/component_measures?id=apache_hive&pullRequest=4123&metric=duplicated_lines_density&view=list) No Duplication information
   
   


-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: gitbox-unsubscribe@hive.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


---------------------------------------------------------------------
To unsubscribe, e-mail: gitbox-unsubscribe@hive.apache.org
For additional commands, e-mail: gitbox-help@hive.apache.org


[GitHub] [hive] wecharyu commented on a diff in pull request #4123: HIVE-27150: Drop single partition can also support direct sql

Posted by "wecharyu (via GitHub)" <gi...@apache.org>.
wecharyu commented on code in PR #4123:
URL: https://github.com/apache/hive/pull/4123#discussion_r1150092003


##########
standalone-metastore/metastore-server/src/main/java/org/apache/hadoop/hive/metastore/ObjectStore.java:
##########
@@ -3101,6 +3100,22 @@ public boolean dropPartition(String catName, String dbName, String tableName,
     return success;
   }
 
+  @Override
+  public boolean dropPartition(String catName, String dbName, String tableName, String partName)
+      throws MetaException, NoSuchObjectException, InvalidObjectException, InvalidInputException {
+    boolean success = false;
+    try {
+      openTransaction();
+      dropPartitionsInternal(catName, dbName, tableName, Arrays.asList(partName), true, true);

Review Comment:
   We have some test for direct sql and jdo sql before, it shows direct sql is always faster even for simple operation like get connection. Let me add the benchmark for dropping multiple single partitions. 



-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: gitbox-unsubscribe@hive.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


---------------------------------------------------------------------
To unsubscribe, e-mail: gitbox-unsubscribe@hive.apache.org
For additional commands, e-mail: gitbox-help@hive.apache.org


[GitHub] [hive] deniskuzZ commented on a diff in pull request #4123: HIVE-27150: Drop single partition can also support direct sql

Posted by "deniskuzZ (via GitHub)" <gi...@apache.org>.
deniskuzZ commented on code in PR #4123:
URL: https://github.com/apache/hive/pull/4123#discussion_r1148088843


##########
standalone-metastore/metastore-server/src/main/java/org/apache/hadoop/hive/metastore/RawStore.java:
##########
@@ -459,16 +459,15 @@ boolean doesPartitionExist(String catName, String dbName, String tableName,
    * @param catName catalog name.
    * @param dbName database name.
    * @param tableName table name.
-   * @param part_vals list of partition values.
+   * @param partName partition name.
    * @return true if the partition was dropped.
    * @throws MetaException Error accessing the RDBMS.
    * @throws NoSuchObjectException no partition matching this description exists
    * @throws InvalidObjectException error dropping the statistics for the partition
    * @throws InvalidInputException error dropping the statistics for the partition
    */
-  boolean dropPartition(String catName, String dbName, String tableName,
-      List<String> part_vals) throws MetaException, NoSuchObjectException, InvalidObjectException,
-      InvalidInputException;
+  boolean dropPartition(String catName, String dbName, String tableName, String partName)

Review Comment:
   That is a change in API that breaks backward compatibility.



-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: gitbox-unsubscribe@hive.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


---------------------------------------------------------------------
To unsubscribe, e-mail: gitbox-unsubscribe@hive.apache.org
For additional commands, e-mail: gitbox-help@hive.apache.org


[GitHub] [hive] VenuReddy2103 commented on a diff in pull request #4123: HIVE-27150: Drop single partition can also support direct sql

Posted by "VenuReddy2103 (via GitHub)" <gi...@apache.org>.
VenuReddy2103 commented on code in PR #4123:
URL: https://github.com/apache/hive/pull/4123#discussion_r1150463962


##########
standalone-metastore/metastore-server/src/main/java/org/apache/hadoop/hive/metastore/RawStore.java:
##########
@@ -459,16 +459,15 @@ boolean doesPartitionExist(String catName, String dbName, String tableName,
    * @param catName catalog name.
    * @param dbName database name.
    * @param tableName table name.
-   * @param part_vals list of partition values.
+   * @param partName partition name.
    * @return true if the partition was dropped.
    * @throws MetaException Error accessing the RDBMS.
    * @throws NoSuchObjectException no partition matching this description exists
    * @throws InvalidObjectException error dropping the statistics for the partition
    * @throws InvalidInputException error dropping the statistics for the partition
    */
-  boolean dropPartition(String catName, String dbName, String tableName,
-      List<String> part_vals) throws MetaException, NoSuchObjectException, InvalidObjectException,
-      InvalidInputException;
+  boolean dropPartition(String catName, String dbName, String tableName, String partName)

Review Comment:
   IMHO, Instead of defining this new API, we can make the partname inside the existing `dropPartition()` method itself to invoke `dropPartitionsInternal()` or `dropPartitions()`. Because the new API signature is similar to exisiting `dropPartitions()` except the last argument(i.e., single partname vs list of partnames). May mark `dropPartition()` as deprecated and insist `dropPartitions()` directly in future.



-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: gitbox-unsubscribe@hive.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


---------------------------------------------------------------------
To unsubscribe, e-mail: gitbox-unsubscribe@hive.apache.org
For additional commands, e-mail: gitbox-help@hive.apache.org


[GitHub] [hive] wecharyu commented on a diff in pull request #4123: HIVE-27150: Drop single partition can also support direct sql

Posted by "wecharyu (via GitHub)" <gi...@apache.org>.
wecharyu commented on code in PR #4123:
URL: https://github.com/apache/hive/pull/4123#discussion_r1150994392


##########
standalone-metastore/metastore-server/src/main/java/org/apache/hadoop/hive/metastore/RawStore.java:
##########
@@ -459,16 +459,15 @@ boolean doesPartitionExist(String catName, String dbName, String tableName,
    * @param catName catalog name.
    * @param dbName database name.
    * @param tableName table name.
-   * @param part_vals list of partition values.
+   * @param partName partition name.
    * @return true if the partition was dropped.
    * @throws MetaException Error accessing the RDBMS.
    * @throws NoSuchObjectException no partition matching this description exists
    * @throws InvalidObjectException error dropping the statistics for the partition
    * @throws InvalidInputException error dropping the statistics for the partition
    */
-  boolean dropPartition(String catName, String dbName, String tableName,
-      List<String> part_vals) throws MetaException, NoSuchObjectException, InvalidObjectException,
-      InvalidInputException;
+  boolean dropPartition(String catName, String dbName, String tableName, String partName)

Review Comment:
   If we reuse the existing `dropPartition()` method, we need to get the `table` and partition keys again in `ObjectStore` to make the partname, so I think it's reasonable to create this new API and deprecate the old one.



-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: gitbox-unsubscribe@hive.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


---------------------------------------------------------------------
To unsubscribe, e-mail: gitbox-unsubscribe@hive.apache.org
For additional commands, e-mail: gitbox-help@hive.apache.org


[GitHub] [hive] deniskuzZ merged pull request #4123: HIVE-27150: Drop single partition can also support direct sql

Posted by "deniskuzZ (via GitHub)" <gi...@apache.org>.
deniskuzZ merged PR #4123:
URL: https://github.com/apache/hive/pull/4123


-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: gitbox-unsubscribe@hive.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


---------------------------------------------------------------------
To unsubscribe, e-mail: gitbox-unsubscribe@hive.apache.org
For additional commands, e-mail: gitbox-help@hive.apache.org


[GitHub] [hive] saihemanth-cloudera commented on a diff in pull request #4123: HIVE-27150: Drop single partition can also support direct sql

Posted by "saihemanth-cloudera (via GitHub)" <gi...@apache.org>.
saihemanth-cloudera commented on code in PR #4123:
URL: https://github.com/apache/hive/pull/4123#discussion_r1162150844


##########
standalone-metastore/metastore-server/src/test/java/org/apache/hadoop/hive/metastore/TestObjectStore.java:
##########
@@ -498,6 +497,68 @@ public void testPartitionOpsWhenTableDoesNotExist() throws InvalidObjectExceptio
     }
   }
 
+  @Test
+  public void testDropPartitionByName() throws Exception {
+    Database db1 = new DatabaseBuilder()
+        .setName(DB1)
+        .setDescription("description")
+        .setLocation("locationurl")
+        .build(conf);
+    try (AutoCloseable c = deadline()) {
+      objectStore.createDatabase(db1);
+    }
+    StorageDescriptor sd = createFakeSd("location");
+    HashMap<String, String> tableParams = new HashMap<>();
+    tableParams.put("EXTERNAL", "false");
+    FieldSchema partitionKey1 = new FieldSchema("Country", ColumnType.STRING_TYPE_NAME, "");
+    FieldSchema partitionKey2 = new FieldSchema("State", ColumnType.STRING_TYPE_NAME, "");
+    Table tbl1 =
+        new Table(TABLE1, DB1, "owner", 1, 2, 3, sd, Arrays.asList(partitionKey1, partitionKey2),
+            tableParams, null, null, "MANAGED_TABLE");
+    try (AutoCloseable c = deadline()) {
+      objectStore.createTable(tbl1);
+    }
+    HashMap<String, String> partitionParams = new HashMap<>();
+    partitionParams.put("PARTITION_LEVEL_PRIVILEGE", "true");
+    List<String> value1 = Arrays.asList("US", "CA");
+    Partition part1 = new Partition(value1, DB1, TABLE1, 111, 111, sd, partitionParams);
+    part1.setCatName(DEFAULT_CATALOG_NAME);
+    try (AutoCloseable c = deadline()) {
+      objectStore.addPartition(part1);
+    }
+    List<String> value2 = Arrays.asList("US", "MA");
+    Partition part2 = new Partition(value2, DB1, TABLE1, 222, 222, sd, partitionParams);
+    part2.setCatName(DEFAULT_CATALOG_NAME);
+    try (AutoCloseable c = deadline()) {
+      objectStore.addPartition(part2);
+    }
+
+    List<Partition> partitions;
+    try (AutoCloseable c = deadline()) {
+      objectStore.dropPartition(DEFAULT_CATALOG_NAME, DB1, TABLE1, "country=US/state=CA");
+      partitions = objectStore.getPartitions(DEFAULT_CATALOG_NAME, DB1, TABLE1, 10);
+    }
+    Assert.assertEquals(1, partitions.size());
+    Assert.assertEquals(222, partitions.get(0).getCreateTime());
+    try (AutoCloseable c = deadline()) {
+      objectStore.dropPartition(DEFAULT_CATALOG_NAME, DB1, TABLE1, "country=US/state=MA");
+      partitions = objectStore.getPartitions(DEFAULT_CATALOG_NAME, DB1, TABLE1, 10);
+    }
+    Assert.assertEquals(0, partitions.size());
+
+    try (AutoCloseable c = deadline()) {
+      // Illegal partName will do nothing, it doesn't matter
+      // because the real HMSHandler will guarantee the partName is legal and exists.
+      objectStore.dropPartition(DEFAULT_CATALOG_NAME, DB1, TABLE1, "country=US/state=NON_EXIST");
+      objectStore.dropPartition(DEFAULT_CATALOG_NAME, DB1, TABLE1, "country=US/st=CA");

Review Comment:
   If this API is returning false, we could assert false here.



-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: gitbox-unsubscribe@hive.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


---------------------------------------------------------------------
To unsubscribe, e-mail: gitbox-unsubscribe@hive.apache.org
For additional commands, e-mail: gitbox-help@hive.apache.org


[GitHub] [hive] wecharyu commented on pull request #4123: HIVE-27150: Drop single partition can also support direct sql

Posted by "wecharyu (via GitHub)" <gi...@apache.org>.
wecharyu commented on PR #4123:
URL: https://github.com/apache/hive/pull/4123#issuecomment-1487395921

   Put the benchmark in PR description, it shows some performance improvement. FYI: @deniskuzZ @saihemanth-cloudera @VenuReddy2103 


-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: gitbox-unsubscribe@hive.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


---------------------------------------------------------------------
To unsubscribe, e-mail: gitbox-unsubscribe@hive.apache.org
For additional commands, e-mail: gitbox-help@hive.apache.org


[GitHub] [hive] sonarcloud[bot] commented on pull request #4123: HIVE-27150: Drop single partition can also support direct sql

Posted by "sonarcloud[bot] (via GitHub)" <gi...@apache.org>.
sonarcloud[bot] commented on PR #4123:
URL: https://github.com/apache/hive/pull/4123#issuecomment-1487528392

   Kudos, SonarCloud Quality Gate passed!&nbsp; &nbsp; [![Quality Gate passed](https://sonarsource.github.io/sonarcloud-github-static-resources/v2/checks/QualityGateBadge/passed-16px.png 'Quality Gate passed')](https://sonarcloud.io/dashboard?id=apache_hive&pullRequest=4123)
   
   [![Bug](https://sonarsource.github.io/sonarcloud-github-static-resources/v2/common/bug-16px.png 'Bug')](https://sonarcloud.io/project/issues?id=apache_hive&pullRequest=4123&resolved=false&types=BUG) [![A](https://sonarsource.github.io/sonarcloud-github-static-resources/v2/checks/RatingBadge/A-16px.png 'A')](https://sonarcloud.io/project/issues?id=apache_hive&pullRequest=4123&resolved=false&types=BUG) [0 Bugs](https://sonarcloud.io/project/issues?id=apache_hive&pullRequest=4123&resolved=false&types=BUG)  
   [![Vulnerability](https://sonarsource.github.io/sonarcloud-github-static-resources/v2/common/vulnerability-16px.png 'Vulnerability')](https://sonarcloud.io/project/issues?id=apache_hive&pullRequest=4123&resolved=false&types=VULNERABILITY) [![A](https://sonarsource.github.io/sonarcloud-github-static-resources/v2/checks/RatingBadge/A-16px.png 'A')](https://sonarcloud.io/project/issues?id=apache_hive&pullRequest=4123&resolved=false&types=VULNERABILITY) [0 Vulnerabilities](https://sonarcloud.io/project/issues?id=apache_hive&pullRequest=4123&resolved=false&types=VULNERABILITY)  
   [![Security Hotspot](https://sonarsource.github.io/sonarcloud-github-static-resources/v2/common/security_hotspot-16px.png 'Security Hotspot')](https://sonarcloud.io/project/security_hotspots?id=apache_hive&pullRequest=4123&resolved=false&types=SECURITY_HOTSPOT) [![A](https://sonarsource.github.io/sonarcloud-github-static-resources/v2/checks/RatingBadge/A-16px.png 'A')](https://sonarcloud.io/project/security_hotspots?id=apache_hive&pullRequest=4123&resolved=false&types=SECURITY_HOTSPOT) [0 Security Hotspots](https://sonarcloud.io/project/security_hotspots?id=apache_hive&pullRequest=4123&resolved=false&types=SECURITY_HOTSPOT)  
   [![Code Smell](https://sonarsource.github.io/sonarcloud-github-static-resources/v2/common/code_smell-16px.png 'Code Smell')](https://sonarcloud.io/project/issues?id=apache_hive&pullRequest=4123&resolved=false&types=CODE_SMELL) [![A](https://sonarsource.github.io/sonarcloud-github-static-resources/v2/checks/RatingBadge/A-16px.png 'A')](https://sonarcloud.io/project/issues?id=apache_hive&pullRequest=4123&resolved=false&types=CODE_SMELL) [7 Code Smells](https://sonarcloud.io/project/issues?id=apache_hive&pullRequest=4123&resolved=false&types=CODE_SMELL)
   
   [![No Coverage information](https://sonarsource.github.io/sonarcloud-github-static-resources/v2/checks/CoverageChart/NoCoverageInfo-16px.png 'No Coverage information')](https://sonarcloud.io/component_measures?id=apache_hive&pullRequest=4123&metric=coverage&view=list) No Coverage information  
   [![No Duplication information](https://sonarsource.github.io/sonarcloud-github-static-resources/v2/checks/Duplications/NoDuplicationInfo-16px.png 'No Duplication information')](https://sonarcloud.io/component_measures?id=apache_hive&pullRequest=4123&metric=duplicated_lines_density&view=list) No Duplication information
   
   


-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: gitbox-unsubscribe@hive.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


---------------------------------------------------------------------
To unsubscribe, e-mail: gitbox-unsubscribe@hive.apache.org
For additional commands, e-mail: gitbox-help@hive.apache.org


[GitHub] [hive] wecharyu commented on a diff in pull request #4123: HIVE-27150: Drop single partition can also support direct sql

Posted by "wecharyu (via GitHub)" <gi...@apache.org>.
wecharyu commented on code in PR #4123:
URL: https://github.com/apache/hive/pull/4123#discussion_r1148297352


##########
standalone-metastore/metastore-server/src/main/java/org/apache/hadoop/hive/metastore/RawStore.java:
##########
@@ -459,16 +459,15 @@ boolean doesPartitionExist(String catName, String dbName, String tableName,
    * @param catName catalog name.
    * @param dbName database name.
    * @param tableName table name.
-   * @param part_vals list of partition values.
+   * @param partName partition name.
    * @return true if the partition was dropped.
    * @throws MetaException Error accessing the RDBMS.
    * @throws NoSuchObjectException no partition matching this description exists
    * @throws InvalidObjectException error dropping the statistics for the partition
    * @throws InvalidInputException error dropping the statistics for the partition
    */
-  boolean dropPartition(String catName, String dbName, String tableName,
-      List<String> part_vals) throws MetaException, NoSuchObjectException, InvalidObjectException,
-      InvalidInputException;
+  boolean dropPartition(String catName, String dbName, String tableName, String partName)

Review Comment:
   Make sense, my initial thought was that this is an internal API which is not exposed to clients, so compatibility was not considered. I will follow your comment.



-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: gitbox-unsubscribe@hive.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


---------------------------------------------------------------------
To unsubscribe, e-mail: gitbox-unsubscribe@hive.apache.org
For additional commands, e-mail: gitbox-help@hive.apache.org


[GitHub] [hive] wecharyu commented on a diff in pull request #4123: HIVE-27150: Drop single partition can also support direct sql

Posted by "wecharyu (via GitHub)" <gi...@apache.org>.
wecharyu commented on code in PR #4123:
URL: https://github.com/apache/hive/pull/4123#discussion_r1152866393


##########
standalone-metastore/metastore-server/src/main/java/org/apache/hadoop/hive/metastore/RawStore.java:
##########
@@ -459,16 +459,15 @@ boolean doesPartitionExist(String catName, String dbName, String tableName,
    * @param catName catalog name.
    * @param dbName database name.
    * @param tableName table name.
-   * @param part_vals list of partition values.
+   * @param partName partition name.
    * @return true if the partition was dropped.
    * @throws MetaException Error accessing the RDBMS.
    * @throws NoSuchObjectException no partition matching this description exists
    * @throws InvalidObjectException error dropping the statistics for the partition
    * @throws InvalidInputException error dropping the statistics for the partition
    */
-  boolean dropPartition(String catName, String dbName, String tableName,
-      List<String> part_vals) throws MetaException, NoSuchObjectException, InvalidObjectException,
-      InvalidInputException;
+  boolean dropPartition(String catName, String dbName, String tableName, String partName)

Review Comment:
   @saihemanth-cloudera Sry I did not get your point. Do you mean add this new API is OK? Thanks for your patience.



-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: gitbox-unsubscribe@hive.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


---------------------------------------------------------------------
To unsubscribe, e-mail: gitbox-unsubscribe@hive.apache.org
For additional commands, e-mail: gitbox-help@hive.apache.org


[GitHub] [hive] saihemanth-cloudera commented on a diff in pull request #4123: HIVE-27150: Drop single partition can also support direct sql

Posted by "saihemanth-cloudera (via GitHub)" <gi...@apache.org>.
saihemanth-cloudera commented on code in PR #4123:
URL: https://github.com/apache/hive/pull/4123#discussion_r1152931417


##########
standalone-metastore/metastore-server/src/main/java/org/apache/hadoop/hive/metastore/RawStore.java:
##########
@@ -459,16 +459,15 @@ boolean doesPartitionExist(String catName, String dbName, String tableName,
    * @param catName catalog name.
    * @param dbName database name.
    * @param tableName table name.
-   * @param part_vals list of partition values.
+   * @param partName partition name.
    * @return true if the partition was dropped.
    * @throws MetaException Error accessing the RDBMS.
    * @throws NoSuchObjectException no partition matching this description exists
    * @throws InvalidObjectException error dropping the statistics for the partition
    * @throws InvalidInputException error dropping the statistics for the partition
    */
-  boolean dropPartition(String catName, String dbName, String tableName,
-      List<String> part_vals) throws MetaException, NoSuchObjectException, InvalidObjectException,
-      InvalidInputException;
+  boolean dropPartition(String catName, String dbName, String tableName, String partName)

Review Comment:
   I'm Ok with your current changes(a new api in the raw store). 



-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: gitbox-unsubscribe@hive.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


---------------------------------------------------------------------
To unsubscribe, e-mail: gitbox-unsubscribe@hive.apache.org
For additional commands, e-mail: gitbox-help@hive.apache.org


[GitHub] [hive] sonarcloud[bot] commented on pull request #4123: HIVE-27150: Drop single partition can also support direct sql

Posted by "sonarcloud[bot] (via GitHub)" <gi...@apache.org>.
sonarcloud[bot] commented on PR #4123:
URL: https://github.com/apache/hive/pull/4123#issuecomment-1492604592

   Kudos, SonarCloud Quality Gate passed!&nbsp; &nbsp; [![Quality Gate passed](https://sonarsource.github.io/sonarcloud-github-static-resources/v2/checks/QualityGateBadge/passed-16px.png 'Quality Gate passed')](https://sonarcloud.io/dashboard?id=apache_hive&pullRequest=4123)
   
   [![Bug](https://sonarsource.github.io/sonarcloud-github-static-resources/v2/common/bug-16px.png 'Bug')](https://sonarcloud.io/project/issues?id=apache_hive&pullRequest=4123&resolved=false&types=BUG) [![A](https://sonarsource.github.io/sonarcloud-github-static-resources/v2/checks/RatingBadge/A-16px.png 'A')](https://sonarcloud.io/project/issues?id=apache_hive&pullRequest=4123&resolved=false&types=BUG) [0 Bugs](https://sonarcloud.io/project/issues?id=apache_hive&pullRequest=4123&resolved=false&types=BUG)  
   [![Vulnerability](https://sonarsource.github.io/sonarcloud-github-static-resources/v2/common/vulnerability-16px.png 'Vulnerability')](https://sonarcloud.io/project/issues?id=apache_hive&pullRequest=4123&resolved=false&types=VULNERABILITY) [![A](https://sonarsource.github.io/sonarcloud-github-static-resources/v2/checks/RatingBadge/A-16px.png 'A')](https://sonarcloud.io/project/issues?id=apache_hive&pullRequest=4123&resolved=false&types=VULNERABILITY) [0 Vulnerabilities](https://sonarcloud.io/project/issues?id=apache_hive&pullRequest=4123&resolved=false&types=VULNERABILITY)  
   [![Security Hotspot](https://sonarsource.github.io/sonarcloud-github-static-resources/v2/common/security_hotspot-16px.png 'Security Hotspot')](https://sonarcloud.io/project/security_hotspots?id=apache_hive&pullRequest=4123&resolved=false&types=SECURITY_HOTSPOT) [![A](https://sonarsource.github.io/sonarcloud-github-static-resources/v2/checks/RatingBadge/A-16px.png 'A')](https://sonarcloud.io/project/security_hotspots?id=apache_hive&pullRequest=4123&resolved=false&types=SECURITY_HOTSPOT) [0 Security Hotspots](https://sonarcloud.io/project/security_hotspots?id=apache_hive&pullRequest=4123&resolved=false&types=SECURITY_HOTSPOT)  
   [![Code Smell](https://sonarsource.github.io/sonarcloud-github-static-resources/v2/common/code_smell-16px.png 'Code Smell')](https://sonarcloud.io/project/issues?id=apache_hive&pullRequest=4123&resolved=false&types=CODE_SMELL) [![A](https://sonarsource.github.io/sonarcloud-github-static-resources/v2/checks/RatingBadge/A-16px.png 'A')](https://sonarcloud.io/project/issues?id=apache_hive&pullRequest=4123&resolved=false&types=CODE_SMELL) [7 Code Smells](https://sonarcloud.io/project/issues?id=apache_hive&pullRequest=4123&resolved=false&types=CODE_SMELL)
   
   [![No Coverage information](https://sonarsource.github.io/sonarcloud-github-static-resources/v2/checks/CoverageChart/NoCoverageInfo-16px.png 'No Coverage information')](https://sonarcloud.io/component_measures?id=apache_hive&pullRequest=4123&metric=coverage&view=list) No Coverage information  
   [![No Duplication information](https://sonarsource.github.io/sonarcloud-github-static-resources/v2/checks/Duplications/NoDuplicationInfo-16px.png 'No Duplication information')](https://sonarcloud.io/component_measures?id=apache_hive&pullRequest=4123&metric=duplicated_lines_density&view=list) No Duplication information
   
   


-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: gitbox-unsubscribe@hive.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


---------------------------------------------------------------------
To unsubscribe, e-mail: gitbox-unsubscribe@hive.apache.org
For additional commands, e-mail: gitbox-help@hive.apache.org