Return-Path: X-Original-To: archive-asf-public-internal@cust-asf2.ponee.io Delivered-To: archive-asf-public-internal@cust-asf2.ponee.io Received: from cust-asf.ponee.io (cust-asf.ponee.io [163.172.22.183]) by cust-asf2.ponee.io (Postfix) with ESMTP id DCDE8200B9C for ; Mon, 10 Oct 2016 22:11:53 +0200 (CEST) Received: by cust-asf.ponee.io (Postfix) id DB8EB160AF1; Mon, 10 Oct 2016 20:11:53 +0000 (UTC) Delivered-To: archive-asf-public@cust-asf.ponee.io Received: from mail.apache.org (hermes.apache.org [140.211.11.3]) by cust-asf.ponee.io (Postfix) with SMTP id 49C14160AE1 for ; Mon, 10 Oct 2016 22:11:52 +0200 (CEST) Received: (qmail 69453 invoked by uid 500); 10 Oct 2016 20:11:51 -0000 Mailing-List: contact commits-help@phoenix.apache.org; run by ezmlm Precedence: bulk List-Help: List-Unsubscribe: List-Post: List-Id: Reply-To: dev@phoenix.apache.org Delivered-To: mailing list commits@phoenix.apache.org Received: (qmail 69387 invoked by uid 99); 10 Oct 2016 20:11:51 -0000 Received: from git1-us-west.apache.org (HELO git1-us-west.apache.org) (140.211.11.23) by apache.org (qpsmtpd/0.29) with ESMTP; Mon, 10 Oct 2016 20:11:51 +0000 Received: by git1-us-west.apache.org (ASF Mail Server at git1-us-west.apache.org, from userid 33) id 469A1E08AF; Mon, 10 Oct 2016 20:11:51 +0000 (UTC) Content-Type: text/plain; charset="us-ascii" MIME-Version: 1.0 Content-Transfer-Encoding: 7bit From: jamestaylor@apache.org To: commits@phoenix.apache.org Date: Mon, 10 Oct 2016 20:11:52 -0000 Message-Id: <4a0b093f57f141a58ffd1ad5d0260fac@git.apache.org> In-Reply-To: References: X-Mailer: ASF-Git Admin Mailer Subject: [2/3] phoenix git commit: PHOENIX-3361 Collect stats correct for local indexes archived-at: Mon, 10 Oct 2016 20:11:54 -0000 http://git-wip-us.apache.org/repos/asf/phoenix/blob/a5b710d1/phoenix-core/src/main/java/org/apache/phoenix/schema/stats/GuidePostsInfo.java ---------------------------------------------------------------------- diff --git a/phoenix-core/src/main/java/org/apache/phoenix/schema/stats/GuidePostsInfo.java b/phoenix-core/src/main/java/org/apache/phoenix/schema/stats/GuidePostsInfo.java index 291cb6d..43fe7f3 100644 --- a/phoenix-core/src/main/java/org/apache/phoenix/schema/stats/GuidePostsInfo.java +++ b/phoenix-core/src/main/java/org/apache/phoenix/schema/stats/GuidePostsInfo.java @@ -22,51 +22,53 @@ import java.util.List; import org.apache.hadoop.hbase.io.ImmutableBytesWritable; import org.apache.phoenix.util.ByteUtil; +import org.apache.phoenix.util.SizedUtil; + +import com.google.common.primitives.Longs; /** * A class that holds the guidePosts of a region and also allows combining the * guidePosts of different regions when the GuidePostsInfo is formed for a table. */ public class GuidePostsInfo { + public final static GuidePostsInfo EMPTY_GUIDEPOST = new GuidePostsInfo(new ArrayList(), + new ImmutableBytesWritable(ByteUtil.EMPTY_BYTE_ARRAY), new ArrayList(), 0, 0) { + @Override + public int getEstimatedSize() { + return 0; + } + }; + public final static GuidePostsInfo NO_GUIDEPOST = new GuidePostsInfo(new ArrayList(), + new ImmutableBytesWritable(ByteUtil.EMPTY_BYTE_ARRAY), new ArrayList(), 0, 0) { + @Override + public int getEstimatedSize() { + return 0; + } + }; /** * the total number of guidePosts for the table combining all the guidePosts per region per cf. */ - private ImmutableBytesWritable guidePosts; - + private final ImmutableBytesWritable guidePosts; /** * Maximum length of a guidePost collected */ - private int maxLength; - - public final static GuidePostsInfo NO_GUIDEPOST = new GuidePostsInfo(new ArrayList(), - new ImmutableBytesWritable(ByteUtil.EMPTY_BYTE_ARRAY), new ArrayList(), 0, 0); - - public int getMaxLength() { - return maxLength; - } - + private final int maxLength; /** * Number of guidePosts */ - private int guidePostsCount; - + private final int guidePostsCount; /** * The rowCounts of each guidePost traversed */ - private List rowCounts; - + private final long[] rowCounts; /** * The bytecounts of each guidePost traversed */ - private List byteCounts; - - public List getRowCounts() { - return rowCounts; - } - - public List getByteCounts() { - return byteCounts; - } + private final long[] byteCounts; + /** + * Estimate of byte size of this instance + */ + private final int estimatedSize; /** * Constructor that creates GuidePostsInfo per region @@ -87,8 +89,16 @@ public class GuidePostsInfo { this.guidePosts = new ImmutableBytesWritable(guidePosts); this.maxLength = maxLength; this.guidePostsCount = guidePostsCount; - this.rowCounts = rowCounts; - this.byteCounts = byteCounts; + this.rowCounts = Longs.toArray(rowCounts); + this.byteCounts = Longs.toArray(byteCounts); + int estimatedSize = SizedUtil.OBJECT_SIZE + + SizedUtil.IMMUTABLE_BYTES_WRITABLE_SIZE + guidePosts.getLength() // guidePosts + + SizedUtil.INT_SIZE // maxLength + + SizedUtil.INT_SIZE // guidePostsCount + + SizedUtil.ARRAY_SIZE + this.rowCounts.length * SizedUtil.LONG_SIZE // rowCounts + + SizedUtil.ARRAY_SIZE + this.byteCounts.length * SizedUtil.LONG_SIZE // byteCounts + + SizedUtil.INT_SIZE; // estimatedSize + this.estimatedSize = estimatedSize; } public ImmutableBytesWritable getGuidePosts() { @@ -98,5 +108,20 @@ public class GuidePostsInfo { public int getGuidePostsCount() { return guidePostsCount; } + + public int getMaxLength() { + return maxLength; + } + public long[] getRowCounts() { + return rowCounts; + } + + public long[] getByteCounts() { + return byteCounts; + } + + public int getEstimatedSize() { + return estimatedSize; + } } \ No newline at end of file http://git-wip-us.apache.org/repos/asf/phoenix/blob/a5b710d1/phoenix-core/src/main/java/org/apache/phoenix/schema/stats/GuidePostsInfoBuilder.java ---------------------------------------------------------------------- diff --git a/phoenix-core/src/main/java/org/apache/phoenix/schema/stats/GuidePostsInfoBuilder.java b/phoenix-core/src/main/java/org/apache/phoenix/schema/stats/GuidePostsInfoBuilder.java index 435f30a..246ef6c 100644 --- a/phoenix-core/src/main/java/org/apache/phoenix/schema/stats/GuidePostsInfoBuilder.java +++ b/phoenix-core/src/main/java/org/apache/phoenix/schema/stats/GuidePostsInfoBuilder.java @@ -51,6 +51,10 @@ public class GuidePostsInfoBuilder { private List rowCounts = new ArrayList(); private List byteCounts = new ArrayList(); + public boolean isEmpty() { + return rowCounts.size() == 0; + } + public List getRowCounts() { return rowCounts; } http://git-wip-us.apache.org/repos/asf/phoenix/blob/a5b710d1/phoenix-core/src/main/java/org/apache/phoenix/schema/stats/GuidePostsKey.java ---------------------------------------------------------------------- diff --git a/phoenix-core/src/main/java/org/apache/phoenix/schema/stats/GuidePostsKey.java b/phoenix-core/src/main/java/org/apache/phoenix/schema/stats/GuidePostsKey.java new file mode 100644 index 0000000..3bde602 --- /dev/null +++ b/phoenix-core/src/main/java/org/apache/phoenix/schema/stats/GuidePostsKey.java @@ -0,0 +1,84 @@ +/* + * 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.phoenix.schema.stats; + +import java.util.Arrays; + +import javax.annotation.Nonnull; + +import org.apache.hadoop.hbase.util.Bytes; + +import com.google.common.base.Preconditions; + +/** + * + * Key for the client-side caching of the guideposts information + * + */ +public final class GuidePostsKey { + private final int hashCode; + @Nonnull private final byte[] physicalName; + @Nonnull private final byte[] columnFamily; + + public GuidePostsKey(byte[] physicalName, byte[] columnFamily) { + Preconditions.checkNotNull(physicalName); + Preconditions.checkNotNull(columnFamily); + this.physicalName = physicalName; + this.columnFamily = columnFamily; + this.hashCode = computeHashCode(); + } + + public byte[] getPhysicalName() { + return physicalName; + } + + public byte[] getColumnFamily() { + return columnFamily; + } + + @Override + public int hashCode() { + return hashCode; + } + + private int computeHashCode() { + final int prime = 31; + int result = 1; + result = prime * result + Arrays.hashCode(columnFamily); + result = prime * result + Arrays.hashCode(physicalName); + return result; + } + + @Override + public boolean equals(Object obj) { + if (this == obj) return true; + if (obj == null) return false; + if (getClass() != obj.getClass()) return false; + GuidePostsKey other = (GuidePostsKey)obj; + if (other.hashCode != this.hashCode) return false; + if (!Arrays.equals(columnFamily, other.columnFamily)) return false; + if (!Arrays.equals(physicalName, other.physicalName)) return false; + return true; + } + + @Override + public String toString() { + return "GuidePostsKey[physicalName=" + Bytes.toStringBinary(physicalName) + + ",columnFamily=" + Bytes.toStringBinary(columnFamily) + "]"; + } +} http://git-wip-us.apache.org/repos/asf/phoenix/blob/a5b710d1/phoenix-core/src/main/java/org/apache/phoenix/schema/stats/NoOpStatisticsCollector.java ---------------------------------------------------------------------- diff --git a/phoenix-core/src/main/java/org/apache/phoenix/schema/stats/NoOpStatisticsCollector.java b/phoenix-core/src/main/java/org/apache/phoenix/schema/stats/NoOpStatisticsCollector.java index a16a48a..74d1710 100644 --- a/phoenix-core/src/main/java/org/apache/phoenix/schema/stats/NoOpStatisticsCollector.java +++ b/phoenix-core/src/main/java/org/apache/phoenix/schema/stats/NoOpStatisticsCollector.java @@ -21,6 +21,7 @@ import java.io.IOException; import java.util.List; import org.apache.hadoop.hbase.Cell; +import org.apache.hadoop.hbase.client.Scan; import org.apache.hadoop.hbase.coprocessor.RegionCoprocessorEnvironment; import org.apache.hadoop.hbase.regionserver.InternalScanner; import org.apache.hadoop.hbase.regionserver.Region; @@ -44,7 +45,7 @@ public class NoOpStatisticsCollector implements StatisticsCollector { } @Override - public void updateStatistic(Region region) { + public void updateStatistic(Region region, Scan scan) { // No-op } http://git-wip-us.apache.org/repos/asf/phoenix/blob/a5b710d1/phoenix-core/src/main/java/org/apache/phoenix/schema/stats/PTableStats.java ---------------------------------------------------------------------- diff --git a/phoenix-core/src/main/java/org/apache/phoenix/schema/stats/PTableStats.java b/phoenix-core/src/main/java/org/apache/phoenix/schema/stats/PTableStats.java deleted file mode 100644 index f297b3b..0000000 --- a/phoenix-core/src/main/java/org/apache/phoenix/schema/stats/PTableStats.java +++ /dev/null @@ -1,57 +0,0 @@ -/* - * Licensed to the Apache Software Foundation (ASF) under one - * or more contributor license agreements. See the NOTICE file - * distributed with this work for additional information - * regarding copyright ownership. The ASF licenses this file - * to you under the Apache License, Version 2.0 (the - * "License"); you may not use this file except in compliance - * with the License. You may obtain a copy of the License at - * - * http://www.apache.org/licenses/LICENSE-2.0 - * - * Unless required by applicable law or agreed to in writing, software - * distributed under the License is distributed on an "AS IS" BASIS, - * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied. - * See the License for the specific language governing permissions and - * limitations under the License. - */ -package org.apache.phoenix.schema.stats; - -import java.util.SortedMap; - -import com.google.common.collect.ImmutableSortedMap; - - -/* - * The table is defined on the client side, but it is populated on the server side. The client should not populate any data to the - * statistics object. - */ -public interface PTableStats { - public static final PTableStats EMPTY_STATS = new PTableStats() { - @Override - public SortedMap getGuidePosts() { - return ImmutableSortedMap.of(); - } - - @Override - public int getEstimatedSize() { - return 0; - } - - @Override - public long getTimestamp() { - return DefaultStatisticsCollector.NO_TIMESTAMP; - } - }; - - /** - * TODO: Change from TreeMap to Map - * Returns a tree map of the guide posts collected against a column family - * @return - */ - SortedMap getGuidePosts(); - - int getEstimatedSize(); - - long getTimestamp(); -} http://git-wip-us.apache.org/repos/asf/phoenix/blob/a5b710d1/phoenix-core/src/main/java/org/apache/phoenix/schema/stats/PTableStatsImpl.java ---------------------------------------------------------------------- diff --git a/phoenix-core/src/main/java/org/apache/phoenix/schema/stats/PTableStatsImpl.java b/phoenix-core/src/main/java/org/apache/phoenix/schema/stats/PTableStatsImpl.java deleted file mode 100644 index 5eb02fd..0000000 --- a/phoenix-core/src/main/java/org/apache/phoenix/schema/stats/PTableStatsImpl.java +++ /dev/null @@ -1,115 +0,0 @@ -/* - * Licensed to the Apache Software Foundation (ASF) under one - * or more contributor license agreements. See the NOTICE file - * distributed with this work for additional information - * regarding copyright ownership. The ASF licenses this file - * to you under the Apache License, Version 2.0 (the - * "License"); you may not use this file except in compliance - * with the License. You may obtain a copy of the License at - * - * http://www.apache.org/licenses/LICENSE-2.0 - * - * Unless required by applicable law or agreed to in writing, software - * distributed under the License is distributed on an "AS IS" BASIS, - * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied. - * See the License for the specific language governing permissions and - * limitations under the License. - */ -package org.apache.phoenix.schema.stats; - -import java.io.ByteArrayInputStream; -import java.io.DataInput; -import java.io.DataInputStream; -import java.io.EOFException; -import java.util.Map; -import java.util.SortedMap; -import java.util.TreeMap; - -import org.apache.hadoop.hbase.io.ImmutableBytesWritable; -import org.apache.hadoop.hbase.util.Bytes; -import org.apache.phoenix.coprocessor.MetaDataProtocol; -import org.apache.phoenix.util.Closeables; -import org.apache.phoenix.util.PrefixByteCodec; -import org.apache.phoenix.util.PrefixByteDecoder; -import org.apache.phoenix.util.SizedUtil; - -import com.sun.istack.NotNull; - - /** - * Implementation for PTableStats. - */ -public class PTableStatsImpl implements PTableStats { - private final SortedMap guidePosts; - private final int estimatedSize; - private final long timeStamp; - - public PTableStatsImpl() { - this(new TreeMap(Bytes.BYTES_COMPARATOR), MetaDataProtocol.MIN_TABLE_TIMESTAMP); - } - - public PTableStatsImpl(@NotNull SortedMap guidePosts, long timeStamp) { - this.guidePosts = guidePosts; - this.timeStamp = timeStamp; - int estimatedSize = SizedUtil.OBJECT_SIZE + SizedUtil.INT_SIZE + SizedUtil.sizeOfTreeMap(guidePosts.size()); - for (Map.Entry entry : guidePosts.entrySet()) { - byte[] cf = entry.getKey(); - estimatedSize += SizedUtil.ARRAY_SIZE + cf.length; - estimatedSize += entry.getValue().getGuidePosts().getLength(); - estimatedSize += SizedUtil.LONG_SIZE; - estimatedSize += SizedUtil.INT_SIZE; - estimatedSize += SizedUtil.INT_SIZE; - } - this.estimatedSize = estimatedSize; - } - - @Override - public SortedMap getGuidePosts() { - return guidePosts; - } - - @Override - public String toString() { - StringBuilder buf = new StringBuilder(); - - buf.append("PTableStats ["); - for (Map.Entry entry : guidePosts.entrySet()) { - buf.append(Bytes.toStringBinary(entry.getKey())); - buf.append(":("); - ImmutableBytesWritable keys = entry.getValue().getGuidePosts(); - ByteArrayInputStream stream = new ByteArrayInputStream(keys.get(), keys.getOffset(), keys.getLength()); - try { - if (keys.getLength() != 0) { - DataInput input = new DataInputStream(stream); - PrefixByteDecoder decoder = new PrefixByteDecoder(entry.getValue().getMaxLength()); - try { - while (true) { - ImmutableBytesWritable ptr = PrefixByteCodec.decode(decoder, input); - buf.append(Bytes.toStringBinary(ptr.get())); - buf.append(","); - } - } catch (EOFException e) { // Ignore as this signifies we're done - - } finally { - Closeables.closeQuietly(stream); - } - buf.setLength(buf.length() - 1); - } - buf.append(")"); - } finally { - Closeables.closeQuietly(stream); - } - } - buf.append("]"); - return buf.toString(); - } - - @Override - public int getEstimatedSize() { - return estimatedSize; - } - - @Override - public long getTimestamp() { - return timeStamp; - } -} http://git-wip-us.apache.org/repos/asf/phoenix/blob/a5b710d1/phoenix-core/src/main/java/org/apache/phoenix/schema/stats/StatisticsCollector.java ---------------------------------------------------------------------- diff --git a/phoenix-core/src/main/java/org/apache/phoenix/schema/stats/StatisticsCollector.java b/phoenix-core/src/main/java/org/apache/phoenix/schema/stats/StatisticsCollector.java index 43ec6c7..60e83a8 100644 --- a/phoenix-core/src/main/java/org/apache/phoenix/schema/stats/StatisticsCollector.java +++ b/phoenix-core/src/main/java/org/apache/phoenix/schema/stats/StatisticsCollector.java @@ -22,6 +22,7 @@ import java.io.IOException; import java.util.List; import org.apache.hadoop.hbase.Cell; +import org.apache.hadoop.hbase.client.Scan; import org.apache.hadoop.hbase.coprocessor.RegionCoprocessorEnvironment; import org.apache.hadoop.hbase.regionserver.InternalScanner; import org.apache.hadoop.hbase.regionserver.Region; @@ -42,9 +43,9 @@ public interface StatisticsCollector extends Closeable { long getMaxTimeStamp(); /** - * Write the collected statistics for the given region. + * Write the collected statistics for the given region over the scan provided. */ - void updateStatistic(Region region); + void updateStatistic(Region region, Scan scan); /** * Collect statistics for the given list of cells. This method can be called multiple times http://git-wip-us.apache.org/repos/asf/phoenix/blob/a5b710d1/phoenix-core/src/main/java/org/apache/phoenix/schema/stats/StatisticsUtil.java ---------------------------------------------------------------------- diff --git a/phoenix-core/src/main/java/org/apache/phoenix/schema/stats/StatisticsUtil.java b/phoenix-core/src/main/java/org/apache/phoenix/schema/stats/StatisticsUtil.java index 685c24e..07b412f 100644 --- a/phoenix-core/src/main/java/org/apache/phoenix/schema/stats/StatisticsUtil.java +++ b/phoenix-core/src/main/java/org/apache/phoenix/schema/stats/StatisticsUtil.java @@ -75,11 +75,11 @@ public class StatisticsUtil { /** Number of parts in our complex key */ protected static final int NUM_KEY_PARTS = 3; - public static byte[] getRowKey(byte[] table, ImmutableBytesPtr fam, byte[] guidePostStartKey) { + public static byte[] getRowKey(byte[] table, ImmutableBytesWritable fam, byte[] guidePostStartKey) { return getRowKey(table, fam, new ImmutableBytesWritable(guidePostStartKey,0,guidePostStartKey.length)); } - public static byte[] getRowKey(byte[] table, ImmutableBytesPtr fam, ImmutableBytesWritable guidePostStartKey) { + public static byte[] getRowKey(byte[] table, ImmutableBytesWritable fam, ImmutableBytesWritable guidePostStartKey) { // always starts with the source table int guidePostLength = guidePostStartKey.getLength(); boolean hasGuidePost = guidePostLength > 0; @@ -97,21 +97,36 @@ public class StatisticsUtil { return rowKey; } - private static byte[] getKey(byte[] table, ImmutableBytesPtr fam) { + private static byte[] getStartKey(byte[] table, ImmutableBytesWritable fam) { + return getKey(table, fam, false); + } + + private static byte[] getEndKey(byte[] table, ImmutableBytesWritable fam) { + byte[] key = getKey(table, fam, true); + ByteUtil.nextKey(key, key.length); + return key; + } + + private static byte[] getKey(byte[] table, ImmutableBytesWritable fam, boolean terminateWithSeparator) { // always starts with the source table and column family - byte[] rowKey = new byte[table.length + fam.getLength() + 1]; + byte[] rowKey = new byte[table.length + fam.getLength() + 1 + (terminateWithSeparator ? 1 : 0)]; int offset = 0; System.arraycopy(table, 0, rowKey, offset, table.length); offset += table.length; rowKey[offset++] = QueryConstants.SEPARATOR_BYTE; // assumes stats table columns not DESC System.arraycopy(fam.get(), fam.getOffset(), rowKey, offset, fam.getLength()); offset += fam.getLength(); + if (terminateWithSeparator) { + rowKey[offset] = QueryConstants.SEPARATOR_BYTE; + } return rowKey; } - private static byte[] getAdjustedKey(byte[] key, byte[] tableNameBytes, ImmutableBytesPtr cf, boolean nextKey) { - if (Bytes.compareTo(key, ByteUtil.EMPTY_BYTE_ARRAY) != 0) { return getRowKey(tableNameBytes, cf, key); } - key = ByteUtil.concat(getKey(tableNameBytes, cf), QueryConstants.SEPARATOR_BYTE_ARRAY); + private static byte[] getAdjustedKey(byte[] key, byte[] tableNameBytes, ImmutableBytesWritable cf, boolean nextKey) { + if (Bytes.compareTo(key, ByteUtil.EMPTY_BYTE_ARRAY) != 0) { + return getRowKey(tableNameBytes, cf, key); + } + key = getKey(tableNameBytes, cf, nextKey); if (nextKey) { ByteUtil.nextKey(key, key.length); } @@ -121,8 +136,10 @@ public class StatisticsUtil { public static List readStatistics(HTableInterface statsHTable, byte[] tableNameBytes, ImmutableBytesPtr cf, byte[] startKey, byte[] stopKey, long clientTimeStamp) throws IOException { List statsForRegion = new ArrayList(); - Scan s = MetaDataUtil.newTableRowsScan(getAdjustedKey(startKey, tableNameBytes, cf, false), - getAdjustedKey(stopKey, tableNameBytes, cf, true), MetaDataProtocol.MIN_TABLE_TIMESTAMP, + Scan s = MetaDataUtil.newTableRowsScan( + getAdjustedKey(startKey, tableNameBytes, cf, false), + getAdjustedKey(stopKey, tableNameBytes, cf, true), + MetaDataProtocol.MIN_TABLE_TIMESTAMP, clientTimeStamp); s.addColumn(QueryConstants.DEFAULT_COLUMN_FAMILY_BYTES, QueryConstants.EMPTY_COLUMN_BYTES); ResultScanner scanner = null; @@ -140,40 +157,27 @@ public class StatisticsUtil { return statsForRegion; } - public static PTableStats readStatistics(HTableInterface statsHTable, byte[] tableNameBytes, long clientTimeStamp) + public static GuidePostsInfo readStatistics(HTableInterface statsHTable, GuidePostsKey key, long clientTimeStamp) throws IOException { ImmutableBytesWritable ptr = new ImmutableBytesWritable(); - Scan s = MetaDataUtil.newTableRowsScan(tableNameBytes, MetaDataProtocol.MIN_TABLE_TIMESTAMP, clientTimeStamp); + ptr.set(key.getColumnFamily()); + byte[] tableNameBytes = key.getPhysicalName(); + byte[] startKey = getStartKey(tableNameBytes, ptr); + byte[] endKey = getEndKey(tableNameBytes, ptr); + Scan s = MetaDataUtil.newTableRowsScan(startKey, endKey, MetaDataProtocol.MIN_TABLE_TIMESTAMP, clientTimeStamp); s.addColumn(QueryConstants.DEFAULT_COLUMN_FAMILY_BYTES, PhoenixDatabaseMetaData.GUIDE_POSTS_WIDTH_BYTES); s.addColumn(QueryConstants.DEFAULT_COLUMN_FAMILY_BYTES, PhoenixDatabaseMetaData.GUIDE_POSTS_ROW_COUNT_BYTES); s.addColumn(QueryConstants.DEFAULT_COLUMN_FAMILY_BYTES, QueryConstants.EMPTY_COLUMN_BYTES); - ResultScanner scanner = null; - long timeStamp = MetaDataProtocol.MIN_TABLE_TIMESTAMP; - TreeMap guidePostsInfoWriterPerCf = new TreeMap(Bytes.BYTES_COMPARATOR); - try { - scanner = statsHTable.getScanner(s); + GuidePostsInfoBuilder guidePostsInfoWriter = new GuidePostsInfoBuilder(); + Cell current = null; + try (ResultScanner scanner = statsHTable.getScanner(s)) { Result result = null; while ((result = scanner.next()) != null) { CellScanner cellScanner = result.cellScanner(); long rowCount = 0; long byteCount = 0; - byte[] cfName = null; - int tableNameLength; - int cfOffset; - int cfLength; - boolean valuesSet = false; - // Only the two cells with quals GUIDE_POSTS_ROW_COUNT_BYTES and GUIDE_POSTS_BYTES would be retrieved - while (cellScanner.advance()) { - Cell current = cellScanner.current(); - if (!valuesSet) { - tableNameLength = tableNameBytes.length + 1; - cfOffset = current.getRowOffset() + tableNameLength; - cfLength = getVarCharLength(current.getRowArray(), cfOffset, - current.getRowLength() - tableNameLength); - ptr.set(current.getRowArray(), cfOffset, cfLength); - valuesSet = true; - } - cfName = ByteUtil.copyKeyBytesIfNecessary(ptr); + while (cellScanner.advance()) { + current = cellScanner.current(); if (Bytes.equals(current.getQualifierArray(), current.getQualifierOffset(), current.getQualifierLength(), PhoenixDatabaseMetaData.GUIDE_POSTS_ROW_COUNT_BYTES, 0, PhoenixDatabaseMetaData.GUIDE_POSTS_ROW_COUNT_BYTES.length)) { @@ -185,28 +189,22 @@ public class StatisticsUtil { byteCount = PLong.INSTANCE.getCodec().decodeLong(current.getValueArray(), current.getValueOffset(), SortOrder.getDefault()); } - if (current.getTimestamp() > timeStamp) { - timeStamp = current.getTimestamp(); - } } - if (cfName != null) { + if (current != null) { + int tableNameLength = tableNameBytes.length + 1; + int cfOffset = current.getRowOffset() + tableNameLength; + int cfLength = getVarCharLength(current.getRowArray(), cfOffset, + current.getRowLength() - tableNameLength); + ptr.set(current.getRowArray(), cfOffset, cfLength); + byte[] cfName = ByteUtil.copyKeyBytesIfNecessary(ptr); byte[] newGPStartKey = getGuidePostsInfoFromRowKey(tableNameBytes, cfName, result.getRow()); - GuidePostsInfoBuilder guidePostsInfoWriter = guidePostsInfoWriterPerCf.get(cfName); - if (guidePostsInfoWriter == null) { - guidePostsInfoWriter = new GuidePostsInfoBuilder(); - guidePostsInfoWriterPerCf.put(cfName, guidePostsInfoWriter); - } guidePostsInfoWriter.addGuidePosts(newGPStartKey, byteCount, rowCount); } } - if (!guidePostsInfoWriterPerCf.isEmpty()) { return new PTableStatsImpl( - getGuidePostsPerCf(guidePostsInfoWriterPerCf), timeStamp); } - } finally { - if (scanner != null) { - scanner.close(); - } } - return PTableStats.EMPTY_STATS; + // We write a row with an empty KeyValue in the case that stats were generated but without enough data + // for any guideposts. If we have no rows, it means stats were never generated. + return current == null ? GuidePostsInfo.NO_GUIDEPOST : guidePostsInfoWriter.isEmpty() ? GuidePostsInfo.EMPTY_GUIDEPOST : guidePostsInfoWriter.build(); } private static SortedMap getGuidePostsPerCf( http://git-wip-us.apache.org/repos/asf/phoenix/blob/a5b710d1/phoenix-core/src/main/java/org/apache/phoenix/schema/stats/StatisticsWriter.java ---------------------------------------------------------------------- diff --git a/phoenix-core/src/main/java/org/apache/phoenix/schema/stats/StatisticsWriter.java b/phoenix-core/src/main/java/org/apache/phoenix/schema/stats/StatisticsWriter.java index b40d34c..fb8d664 100644 --- a/phoenix-core/src/main/java/org/apache/phoenix/schema/stats/StatisticsWriter.java +++ b/phoenix-core/src/main/java/org/apache/phoenix/schema/stats/StatisticsWriter.java @@ -135,8 +135,8 @@ public class StatisticsWriter implements Closeable { } GuidePostsInfo gps = tracker.getGuidePosts(cfKey); if (gps != null) { - List byteCounts = gps.getByteCounts(); - List rowCounts = gps.getRowCounts(); + long[] byteCounts = gps.getByteCounts(); + long[] rowCounts = gps.getRowCounts(); ImmutableBytesWritable keys = gps.getGuidePosts(); boolean hasGuidePosts = keys.getLength() > 0; if (hasGuidePosts) { @@ -146,7 +146,7 @@ public class StatisticsWriter implements Closeable { PrefixByteDecoder decoder = new PrefixByteDecoder(gps.getMaxLength()); do { ImmutableBytesWritable ptr = decoder.decode(input); - addGuidepost(cfKey, mutations, ptr, byteCounts.get(guidePostCount), rowCounts.get(guidePostCount), timeStamp); + addGuidepost(cfKey, mutations, ptr, byteCounts[guidePostCount], rowCounts[guidePostCount], timeStamp); guidePostCount++; } while (decoder != null); } catch (EOFException e) { // Ignore as this signifies we're done http://git-wip-us.apache.org/repos/asf/phoenix/blob/a5b710d1/phoenix-core/src/main/java/org/apache/phoenix/util/MetaDataUtil.java ---------------------------------------------------------------------- diff --git a/phoenix-core/src/main/java/org/apache/phoenix/util/MetaDataUtil.java b/phoenix-core/src/main/java/org/apache/phoenix/util/MetaDataUtil.java index a88aea8..a9bb468 100644 --- a/phoenix-core/src/main/java/org/apache/phoenix/util/MetaDataUtil.java +++ b/phoenix-core/src/main/java/org/apache/phoenix/util/MetaDataUtil.java @@ -638,4 +638,8 @@ public class MetaDataUtil { return cfPtr.getLength() >= QueryConstants.LOCAL_INDEX_COLUMN_FAMILY_PREFIX_BYTES.length && Bytes.compareTo(cfPtr.get(), cfPtr.getOffset(), QueryConstants.LOCAL_INDEX_COLUMN_FAMILY_PREFIX_BYTES.length, QueryConstants.LOCAL_INDEX_COLUMN_FAMILY_PREFIX_BYTES, 0, QueryConstants.LOCAL_INDEX_COLUMN_FAMILY_PREFIX_BYTES.length) == 0; } + + public static boolean isLocalIndexFamily(byte[] cf) { + return Bytes.startsWith(cf, QueryConstants.LOCAL_INDEX_COLUMN_FAMILY_PREFIX_BYTES); + } } http://git-wip-us.apache.org/repos/asf/phoenix/blob/a5b710d1/phoenix-core/src/test/java/org/apache/phoenix/filter/SkipScanBigFilterTest.java ---------------------------------------------------------------------- diff --git a/phoenix-core/src/test/java/org/apache/phoenix/filter/SkipScanBigFilterTest.java b/phoenix-core/src/test/java/org/apache/phoenix/filter/SkipScanBigFilterTest.java index ad99514..9ba7f6d 100644 --- a/phoenix-core/src/test/java/org/apache/phoenix/filter/SkipScanBigFilterTest.java +++ b/phoenix-core/src/test/java/org/apache/phoenix/filter/SkipScanBigFilterTest.java @@ -24,7 +24,6 @@ import java.sql.DriverManager; import java.sql.PreparedStatement; import java.util.Map; import java.util.Properties; -import java.util.SortedMap; import org.apache.hadoop.hbase.util.Bytes; import org.apache.phoenix.compile.QueryPlan; @@ -32,7 +31,6 @@ import org.apache.phoenix.end2end.Shadower; import org.apache.phoenix.jdbc.PhoenixConnection; import org.apache.phoenix.jdbc.PhoenixStatement; import org.apache.phoenix.query.BaseConnectionlessQueryTest; -import org.apache.phoenix.query.ConnectionQueryServicesImpl; import org.apache.phoenix.query.ConnectionlessQueryServicesImpl; import org.apache.phoenix.query.QueryConstants; import org.apache.phoenix.query.QueryServices; @@ -40,7 +38,7 @@ import org.apache.phoenix.schema.PTable; import org.apache.phoenix.schema.PTableKey; import org.apache.phoenix.schema.stats.GuidePostsInfo; import org.apache.phoenix.schema.stats.GuidePostsInfoBuilder; -import org.apache.phoenix.schema.stats.PTableStats; +import org.apache.phoenix.schema.stats.GuidePostsKey; import org.apache.phoenix.util.PropertiesUtil; import org.apache.phoenix.util.ReadOnlyProps; import org.junit.BeforeClass; @@ -653,29 +651,10 @@ public class SkipScanBigFilterTest extends BaseConnectionlessQueryTest { gpWriter.addGuidePosts(gp, 1000); } GuidePostsInfo info = gpWriter.build(); - final SortedMap gpMap = Maps.newTreeMap(Bytes.BYTES_COMPARATOR); - gpMap.put(QueryConstants.DEFAULT_COLUMN_FAMILY_BYTES, info); - PTableStats stats = new PTableStats() { - - @Override - public SortedMap getGuidePosts() { - return gpMap; - } - - @Override - public int getEstimatedSize() { - return 10000; - } - - @Override - public long getTimestamp() { - return table.getTimeStamp()+1; - } - }; PhoenixConnection pConn = conn.unwrap(PhoenixConnection.class); pConn.addTable(table, System.currentTimeMillis()); ((ConnectionlessQueryServicesImpl) pConn.getQueryServices()) - .addTableStats(table.getName().getBytesPtr(), stats); + .addTableStats(new GuidePostsKey(table.getName().getBytes(), QueryConstants.DEFAULT_COLUMN_FAMILY_BYTES), info); String query = "SELECT count(1) cnt,\n" + " coalesce(SUM(impressions), 0.0) AS \"impressions\",\n" + http://git-wip-us.apache.org/repos/asf/phoenix/blob/a5b710d1/phoenix-core/src/test/java/org/apache/phoenix/query/PhoenixStatsCacheRemovalListenerTest.java ---------------------------------------------------------------------- diff --git a/phoenix-core/src/test/java/org/apache/phoenix/query/PhoenixStatsCacheRemovalListenerTest.java b/phoenix-core/src/test/java/org/apache/phoenix/query/PhoenixStatsCacheRemovalListenerTest.java index b4e01a7..7f84219 100644 --- a/phoenix-core/src/test/java/org/apache/phoenix/query/PhoenixStatsCacheRemovalListenerTest.java +++ b/phoenix-core/src/test/java/org/apache/phoenix/query/PhoenixStatsCacheRemovalListenerTest.java @@ -19,7 +19,7 @@ package org.apache.phoenix.query; import static org.junit.Assert.assertFalse; import static org.junit.Assert.assertTrue; -import org.apache.phoenix.query.TableStatsCache.PhoenixStatsCacheRemovalListener; +import org.apache.phoenix.query.GuidePostsCache.PhoenixStatsCacheRemovalListener; import org.junit.Test; import com.google.common.cache.RemovalCause; http://git-wip-us.apache.org/repos/asf/phoenix/blob/a5b710d1/phoenix-core/src/test/java/org/apache/phoenix/util/TestUtil.java ---------------------------------------------------------------------- diff --git a/phoenix-core/src/test/java/org/apache/phoenix/util/TestUtil.java b/phoenix-core/src/test/java/org/apache/phoenix/util/TestUtil.java index de5af46..03469e2 100644 --- a/phoenix-core/src/test/java/org/apache/phoenix/util/TestUtil.java +++ b/phoenix-core/src/test/java/org/apache/phoenix/util/TestUtil.java @@ -52,6 +52,7 @@ import java.util.Properties; import org.apache.commons.logging.Log; import org.apache.commons.logging.LogFactory; import org.apache.hadoop.hbase.Cell; +import org.apache.hadoop.hbase.CellScanner; import org.apache.hadoop.hbase.HConstants; import org.apache.hadoop.hbase.client.Delete; import org.apache.hadoop.hbase.client.HBaseAdmin; @@ -105,11 +106,12 @@ import org.apache.phoenix.query.QueryServicesOptions; import org.apache.phoenix.schema.PColumn; import org.apache.phoenix.schema.PLongColumn; import org.apache.phoenix.schema.PName; +import org.apache.phoenix.schema.PTable; import org.apache.phoenix.schema.RowKeyValueAccessor; import org.apache.phoenix.schema.SortOrder; import org.apache.phoenix.schema.TableRef; import org.apache.phoenix.schema.stats.GuidePostsInfo; -import org.apache.phoenix.schema.stats.PTableStats; +import org.apache.phoenix.schema.stats.GuidePostsKey; import org.apache.phoenix.schema.tuple.Tuple; import org.apache.phoenix.schema.types.PDataType; @@ -595,9 +597,9 @@ public class TestUtil { pstmt.execute(); TableRef tableRef = pstmt.getQueryPlan().getTableRef(); PhoenixConnection pconn = conn.unwrap(PhoenixConnection.class); - long scn = null == pconn.getSCN() ? Long.MAX_VALUE : pconn.getSCN(); - PTableStats tableStats = pconn.getQueryServices().getTableStats(tableRef.getTable().getName().getBytes(), scn); - return tableStats.getGuidePosts().values(); + PTable table = tableRef.getTable(); + GuidePostsInfo info = pconn.getQueryServices().getTableStats(new GuidePostsKey(table.getName().getBytes(), SchemaUtil.getEmptyColumnFamily(table))); + return Collections.singletonList(info); } public static void analyzeTable(Connection conn, String tableName) throws IOException, SQLException { @@ -793,5 +795,23 @@ public class TestUtil { public static void createTransactionalTable(Connection conn, String tableName) throws SQLException { conn.createStatement().execute("create table " + tableName + TestUtil.TEST_TABLE_SCHEMA + "TRANSACTIONAL=true"); } + + public static void dumpTable(HTableInterface table) throws IOException { + System.out.println("************ dumping " + table + " **************"); + Scan s = new Scan(); + try (ResultScanner scanner = table.getScanner(s)) { + Result result = null; + while ((result = scanner.next()) != null) { + CellScanner cellScanner = result.cellScanner(); + Cell current = null; + while (cellScanner.advance()) { + current = cellScanner.current(); + System.out.println(current); + } + } + } + System.out.println("-----------------------------------------------"); + } + }