You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@pinot.apache.org by ki...@apache.org on 2020/12/21 01:20:35 UTC

[incubator-pinot] branch h3-index updated: Using the parameters from the query and functional

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

kishoreg pushed a commit to branch h3-index
in repository https://gitbox.apache.org/repos/asf/incubator-pinot.git


The following commit(s) were added to refs/heads/h3-index by this push:
     new efcec4f  Using the parameters from the query and functional
efcec4f is described below

commit efcec4f7856a270a5017479643e91bc3ca9a66c5
Author: kishoreg <g....@gmail.com>
AuthorDate: Sun Dec 20 17:20:11 2020 -0800

    Using the parameters from the query and functional
---
 .../operator/filter/H3IndexFilterOperator.java     | 50 +++++++++++++++++++---
 .../org/apache/pinot/core/plan/FilterPlanNode.java | 12 ++++++
 .../request/context/predicate/GeoPredicate.java    | 34 ++++++++++++++-
 .../index/readers/geospatial/H3IndexReader.java    |  5 +++
 4 files changed, 95 insertions(+), 6 deletions(-)

diff --git a/pinot-core/src/main/java/org/apache/pinot/core/operator/filter/H3IndexFilterOperator.java b/pinot-core/src/main/java/org/apache/pinot/core/operator/filter/H3IndexFilterOperator.java
index 13e79a7..7528b7e 100644
--- a/pinot-core/src/main/java/org/apache/pinot/core/operator/filter/H3IndexFilterOperator.java
+++ b/pinot-core/src/main/java/org/apache/pinot/core/operator/filter/H3IndexFilterOperator.java
@@ -18,12 +18,17 @@
  */
 package org.apache.pinot.core.operator.filter;
 
+import com.uber.h3core.H3Core;
+import com.uber.h3core.LengthUnit;
+import java.io.IOException;
+import java.util.List;
 import org.apache.pinot.core.common.DataSource;
 import org.apache.pinot.core.operator.blocks.FilterBlock;
 import org.apache.pinot.core.operator.docidsets.BitmapDocIdSet;
 import org.apache.pinot.core.query.request.context.predicate.GeoPredicate;
 import org.apache.pinot.core.segment.index.readers.geospatial.H3IndexReader;
 import org.roaringbitmap.buffer.ImmutableRoaringBitmap;
+import org.roaringbitmap.buffer.MutableRoaringBitmap;
 
 
 public class H3IndexFilterOperator extends BaseFilterOperator {
@@ -34,26 +39,61 @@ public class H3IndexFilterOperator extends BaseFilterOperator {
   private final GeoPredicate _geoPredicate;
   private final DataSource _dataSource;
   private final int _numDocs;
+  private final H3Core _h3Core;
 
   public H3IndexFilterOperator(GeoPredicate geoPredicate, DataSource dataSource, int numDocs) {
     _geoPredicate = geoPredicate;
     _dataSource = dataSource;
     _numDocs = numDocs;
+    try {
+      _h3Core = H3Core.newInstance();
+    } catch (IOException e) {
+      throw new RuntimeException("Unable to instantiate H3", e);      //todo:log error
+    }
   }
 
   @Override
   protected FilterBlock getNextBlock() {
     H3IndexReader h3IndexReader = _dataSource.getH3Index();
+    //todo: this needs to come from somewhere?
+    int resolution = 5;
+    long h3Id = _h3Core
+        .geoToH3(_geoPredicate.getGeometry().getCoordinate().x, _geoPredicate.getGeometry().getCoordinate().y,
+            resolution);
     assert h3IndexReader != null;
-    //todo: pick this from the geoPredicate
-    long h3Id = h3IndexReader.getDictionary().getLongValue(0);
-    ImmutableRoaringBitmap docIds = h3IndexReader.getDocIds(h3Id);
-    return new FilterBlock(new BitmapDocIdSet(docIds, _numDocs) {
+
+    //find the number of rings based on geopredicate.distance
+    //FullMatch
+    double edgeLength = _h3Core.edgeLength(resolution, LengthUnit.km);
+    int numFullMatchedRings = (int) (_geoPredicate.getDistance() / edgeLength);
+    List<Long> fullMatchRings = _h3Core.kRing(h3Id, numFullMatchedRings);
+    fullMatchRings.add(h3Id);
+    MutableRoaringBitmap fullMatchedDocIds = new MutableRoaringBitmap();
+    for (long id : fullMatchRings) {
+      ImmutableRoaringBitmap docIds = h3IndexReader.getDocIds(id);
+      fullMatchedDocIds.or(docIds);
+    }
+
+    //partial matchedRings
+    int numPartialMatchedRings = (int) (_geoPredicate.getDistance() / edgeLength);
+    List<Long> partialMatchedRings = _h3Core.kRing(h3Id, numPartialMatchedRings);
+    partialMatchedRings.add(h3Id);
+    final MutableRoaringBitmap partialMatchDocIds = new MutableRoaringBitmap();
+    partialMatchedRings.removeAll(fullMatchRings);
+    for (long id : partialMatchedRings) {
+      ImmutableRoaringBitmap docIds = h3IndexReader.getDocIds(id);
+      partialMatchDocIds.or(docIds);
+    }
+
+    //TODO:evaluate the actual distance for the partial matched by scanning
+
+    MutableRoaringBitmap result = ImmutableRoaringBitmap.or(fullMatchedDocIds, partialMatchDocIds);
+    return new FilterBlock(new BitmapDocIdSet(result, _numDocs) {
 
       // Override this method to reflect the entries scanned
       @Override
       public long getNumEntriesScannedInFilter() {
-        return 0; //TODO:Return the one from ScanBased
+        return partialMatchDocIds.getCardinality();
       }
     });
   }
diff --git a/pinot-core/src/main/java/org/apache/pinot/core/plan/FilterPlanNode.java b/pinot-core/src/main/java/org/apache/pinot/core/plan/FilterPlanNode.java
index f31fc77..43b2174 100644
--- a/pinot-core/src/main/java/org/apache/pinot/core/plan/FilterPlanNode.java
+++ b/pinot-core/src/main/java/org/apache/pinot/core/plan/FilterPlanNode.java
@@ -24,6 +24,7 @@ import java.util.List;
 import java.util.Map;
 import javax.annotation.Nullable;
 import org.apache.pinot.core.common.DataSource;
+import org.apache.pinot.core.geospatial.GeometryUtils;
 import org.apache.pinot.core.indexsegment.IndexSegment;
 import org.apache.pinot.core.operator.filter.BaseFilterOperator;
 import org.apache.pinot.core.operator.filter.BitmapBasedFilterOperator;
@@ -45,6 +46,9 @@ import org.apache.pinot.core.query.request.context.predicate.TextMatchPredicate;
 import org.apache.pinot.core.segment.index.readers.NullValueVectorReader;
 import org.apache.pinot.core.segment.index.readers.ValidDocIndexReader;
 import org.apache.pinot.core.util.QueryOptions;
+import org.locationtech.jts.geom.Coordinate;
+import org.locationtech.jts.geom.Geometry;
+import org.locationtech.jts.geom.Point;
 
 
 public class FilterPlanNode implements PlanNode {
@@ -127,6 +131,14 @@ public class FilterPlanNode implements PlanNode {
           if (function.getFunctionName().equalsIgnoreCase("H3_WITHIN")) {
             String columnName = function.getArguments().get(0).getIdentifier();
             GeoPredicate geoPredicate = new GeoPredicate();
+            geoPredicate.setType(GeoPredicate.Type.WITHIN);
+            float lat = Float.parseFloat(function.getArguments().get(1).getLiteral());
+            float lon = Float.parseFloat(function.getArguments().get(2).getLiteral());
+            float distance = Float.parseFloat(function.getArguments().get(3).getLiteral());
+//            float resolution =Float.parseFloat(function.getArguments().get(4).getLiteral());
+            Point point = GeometryUtils.GEOMETRY_FACTORY.createPoint(new Coordinate(lat, lon));
+            geoPredicate.setGeometry(point);
+            geoPredicate.setDistance(distance);
             //set geo predicate
             return new H3IndexFilterOperator(geoPredicate, _indexSegment.getDataSource(columnName), _numDocs);
           } else {
diff --git a/pinot-core/src/main/java/org/apache/pinot/core/query/request/context/predicate/GeoPredicate.java b/pinot-core/src/main/java/org/apache/pinot/core/query/request/context/predicate/GeoPredicate.java
index 8f51ae5..f47ace2 100644
--- a/pinot-core/src/main/java/org/apache/pinot/core/query/request/context/predicate/GeoPredicate.java
+++ b/pinot-core/src/main/java/org/apache/pinot/core/query/request/context/predicate/GeoPredicate.java
@@ -16,7 +16,39 @@ public class GeoPredicate {
 
   double _distance;
 
-  enum Type {
+  public enum Type {
     WITHIN, OVERLAP;
   }
+
+  public ExpressionContext getLhs() {
+    return _lhs;
+  }
+
+  public void setLhs(ExpressionContext lhs) {
+    _lhs = lhs;
+  }
+
+  public Type getType() {
+    return type;
+  }
+
+  public void setType(Type type) {
+    this.type = type;
+  }
+
+  public Geometry getGeometry() {
+    return _geometry;
+  }
+
+  public void setGeometry(Geometry geometry) {
+    _geometry = geometry;
+  }
+
+  public double getDistance() {
+    return _distance;
+  }
+
+  public void setDistance(double distance) {
+    _distance = distance;
+  }
 }
diff --git a/pinot-core/src/main/java/org/apache/pinot/core/segment/index/readers/geospatial/H3IndexReader.java b/pinot-core/src/main/java/org/apache/pinot/core/segment/index/readers/geospatial/H3IndexReader.java
index 9470c33..94152f3 100644
--- a/pinot-core/src/main/java/org/apache/pinot/core/segment/index/readers/geospatial/H3IndexReader.java
+++ b/pinot-core/src/main/java/org/apache/pinot/core/segment/index/readers/geospatial/H3IndexReader.java
@@ -7,7 +7,9 @@ import org.apache.pinot.core.segment.index.readers.Dictionary;
 import org.apache.pinot.core.segment.index.readers.IntDictionary;
 import org.apache.pinot.core.segment.index.readers.LongDictionary;
 import org.apache.pinot.core.segment.memory.PinotDataBuffer;
+import org.roaringbitmap.RoaringBitmap;
 import org.roaringbitmap.buffer.ImmutableRoaringBitmap;
+import org.roaringbitmap.buffer.MutableRoaringBitmap;
 import org.slf4j.Logger;
 import org.slf4j.LoggerFactory;
 
@@ -47,6 +49,9 @@ public class H3IndexReader implements Closeable {
   public ImmutableRoaringBitmap getDocIds(long h3IndexId) {
     SoftReference<ImmutableRoaringBitmap>[] bitmapArrayReference = null;
     int dictId = _dictionary.indexOf(String.valueOf(h3IndexId));
+    if (dictId < 0) {
+      return new MutableRoaringBitmap();
+    }
     // Return the bitmap if it's still on heap
     if (_bitmaps != null) {
       bitmapArrayReference = _bitmaps.get();


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