Repository: accumulo Updated Branches: refs/heads/master e605349c0 -> 3f1b0f33c
ACCUMULO-4636 system iterator improvements Project: http://git-wip-us.apache.org/repos/asf/accumulo/repo Commit: http://git-wip-us.apache.org/repos/asf/accumulo/commit/2ff26780 Tree: http://git-wip-us.apache.org/repos/asf/accumulo/tree/2ff26780 Diff: http://git-wip-us.apache.org/repos/asf/accumulo/diff/2ff26780 Branch: refs/heads/master Commit: 2ff26780e77ff87ae62bd85f815655d639cf8519 Parents: 5f6882f Author: Mike Miller <mmil...@apache.org> Authored: Fri May 5 16:21:20 2017 -0400 Committer: Mike Miller <mmil...@apache.org> Committed: Mon May 8 14:55:26 2017 -0400 ---------------------------------------------------------------------- .../core/client/impl/OfflineIterator.java | 2 +- .../core/client/mock/MockScannerBase.java | 4 +- .../apache/accumulo/core/iterators/Filter.java | 5 +- .../core/iterators/FirstEntryInRowIterator.java | 11 ++-- .../system/ColumnFamilySkippingIterator.java | 17 ++--- .../iterators/system/ColumnQualifierFilter.java | 68 +++++++++++--------- .../core/iterators/system/DeletingIterator.java | 28 ++++---- .../core/iterators/system/VisibilityFilter.java | 51 ++++++++++----- .../core/iterators/user/VersioningIterator.java | 5 +- .../core/iterators/user/VisibilityFilter.java | 43 +++++++++---- .../core/iterators/system/ColumnFilterTest.java | 58 ++++++++++++----- .../iterators/system/VisibilityFilterTest.java | 20 +++++- .../core/iterators/user/FilterTest.java | 12 ++-- .../problems/ProblemReportingIterator.java | 4 +- .../apache/accumulo/tserver/InMemoryMap.java | 7 -- .../accumulo/tserver/tablet/ScanDataSource.java | 2 +- .../performance/scan/CollectTabletStats.java | 4 +- 17 files changed, 217 insertions(+), 124 deletions(-) ---------------------------------------------------------------------- http://git-wip-us.apache.org/repos/asf/accumulo/blob/2ff26780/core/src/main/java/org/apache/accumulo/core/client/impl/OfflineIterator.java ---------------------------------------------------------------------- diff --git a/core/src/main/java/org/apache/accumulo/core/client/impl/OfflineIterator.java b/core/src/main/java/org/apache/accumulo/core/client/impl/OfflineIterator.java index adb1f50..21896e6 100644 --- a/core/src/main/java/org/apache/accumulo/core/client/impl/OfflineIterator.java +++ b/core/src/main/java/org/apache/accumulo/core/client/impl/OfflineIterator.java @@ -325,7 +325,7 @@ class OfflineIterator implements Iterator<Entry<Key,Value>> { ColumnVisibility cv = new ColumnVisibility(acuTableConf.get(Property.TABLE_DEFAULT_SCANTIME_VISIBILITY)); defaultSecurityLabel = cv.getExpression(); - VisibilityFilter visFilter = new VisibilityFilter(colFilter, authorizations, defaultSecurityLabel); + SortedKeyValueIterator<Key,Value> visFilter = VisibilityFilter.wrap(colFilter, authorizations, defaultSecurityLabel); return iterEnv.getTopLevelIterator(IteratorUtil.loadIterators(IteratorScope.scan, visFilter, extent, acuTableConf, options.serverSideIteratorList, options.serverSideIteratorOptions, iterEnv, false)); http://git-wip-us.apache.org/repos/asf/accumulo/blob/2ff26780/core/src/main/java/org/apache/accumulo/core/client/mock/MockScannerBase.java ---------------------------------------------------------------------- diff --git a/core/src/main/java/org/apache/accumulo/core/client/mock/MockScannerBase.java b/core/src/main/java/org/apache/accumulo/core/client/mock/MockScannerBase.java index 0a31bf2..8b612d5 100644 --- a/core/src/main/java/org/apache/accumulo/core/client/mock/MockScannerBase.java +++ b/core/src/main/java/org/apache/accumulo/core/client/mock/MockScannerBase.java @@ -113,8 +113,8 @@ public class MockScannerBase extends ScannerOptions implements ScannerBase { public SortedKeyValueIterator<Key,Value> createFilter(SortedKeyValueIterator<Key,Value> inner) throws IOException { byte[] defaultLabels = {}; inner = new ColumnFamilySkippingIterator(new DeletingIterator(inner, false)); - ColumnQualifierFilter cqf = new ColumnQualifierFilter(inner, new HashSet<>(fetchedColumns)); - VisibilityFilter vf = new VisibilityFilter(cqf, auths, defaultLabels); + SortedKeyValueIterator<Key,Value> cqf = ColumnQualifierFilter.wrap(inner, new HashSet<>(fetchedColumns)); + SortedKeyValueIterator<Key,Value> vf = VisibilityFilter.wrap(cqf, auths, defaultLabels); AccumuloConfiguration conf = new MockConfiguration(table.settings); MockIteratorEnvironment iterEnv = new MockIteratorEnvironment(auths); SortedKeyValueIterator<Key,Value> result = iterEnv.getTopLevelIterator(IteratorUtil.loadIterators(IteratorScope.scan, vf, null, conf, http://git-wip-us.apache.org/repos/asf/accumulo/blob/2ff26780/core/src/main/java/org/apache/accumulo/core/iterators/Filter.java ---------------------------------------------------------------------- diff --git a/core/src/main/java/org/apache/accumulo/core/iterators/Filter.java b/core/src/main/java/org/apache/accumulo/core/iterators/Filter.java index 8b135c7..e2d13d6 100644 --- a/core/src/main/java/org/apache/accumulo/core/iterators/Filter.java +++ b/core/src/main/java/org/apache/accumulo/core/iterators/Filter.java @@ -69,9 +69,10 @@ public abstract class Filter extends WrappingIterator implements OptionDescriber * Iterates over the source until an acceptable key/value pair is found. */ protected void findTop() { - while (getSource().hasTop() && !getSource().getTopKey().isDeleted() && (negate == accept(getSource().getTopKey(), getSource().getTopValue()))) { + SortedKeyValueIterator<Key,Value> source = getSource(); + while (source.hasTop() && !source.getTopKey().isDeleted() && (negate == accept(source.getTopKey(), source.getTopValue()))) { try { - getSource().next(); + source.next(); } catch (IOException e) { throw new RuntimeException(e); } http://git-wip-us.apache.org/repos/asf/accumulo/blob/2ff26780/core/src/main/java/org/apache/accumulo/core/iterators/FirstEntryInRowIterator.java ---------------------------------------------------------------------- diff --git a/core/src/main/java/org/apache/accumulo/core/iterators/FirstEntryInRowIterator.java b/core/src/main/java/org/apache/accumulo/core/iterators/FirstEntryInRowIterator.java index 32e6464..8892d66 100644 --- a/core/src/main/java/org/apache/accumulo/core/iterators/FirstEntryInRowIterator.java +++ b/core/src/main/java/org/apache/accumulo/core/iterators/FirstEntryInRowIterator.java @@ -78,27 +78,28 @@ public class FirstEntryInRowIterator extends SkippingIterator implements OptionD if (finished == true || lastRowFound == null) return; int count = 0; - while (getSource().hasTop() && lastRowFound.equals(getSource().getTopKey().getRow())) { + SortedKeyValueIterator<Key,Value> source = getSource(); + while (source.hasTop() && lastRowFound.equals(source.getTopKey().getRow())) { // try to efficiently jump to the next matching key if (count < numscans) { ++count; - getSource().next(); // scan + source.next(); // scan } else { // too many scans, just seek count = 0; // determine where to seek to, but don't go beyond the user-specified range - Key nextKey = getSource().getTopKey().followingKey(PartialKey.ROW); + Key nextKey = source.getTopKey().followingKey(PartialKey.ROW); if (!latestRange.afterEndKey(nextKey)) - getSource().seek(new Range(nextKey, true, latestRange.getEndKey(), latestRange.isEndKeyInclusive()), latestColumnFamilies, latestInclusive); + source.seek(new Range(nextKey, true, latestRange.getEndKey(), latestRange.isEndKeyInclusive()), latestColumnFamilies, latestInclusive); else { finished = true; break; } } } - lastRowFound = getSource().hasTop() ? getSource().getTopKey().getRow(lastRowFound) : null; + lastRowFound = source.hasTop() ? source.getTopKey().getRow(lastRowFound) : null; } private boolean finished = true; http://git-wip-us.apache.org/repos/asf/accumulo/blob/2ff26780/core/src/main/java/org/apache/accumulo/core/iterators/system/ColumnFamilySkippingIterator.java ---------------------------------------------------------------------- diff --git a/core/src/main/java/org/apache/accumulo/core/iterators/system/ColumnFamilySkippingIterator.java b/core/src/main/java/org/apache/accumulo/core/iterators/system/ColumnFamilySkippingIterator.java index 53f3643..ad4bf90 100644 --- a/core/src/main/java/org/apache/accumulo/core/iterators/system/ColumnFamilySkippingIterator.java +++ b/core/src/main/java/org/apache/accumulo/core/iterators/system/ColumnFamilySkippingIterator.java @@ -52,36 +52,37 @@ public class ColumnFamilySkippingIterator extends SkippingIterator implements In @Override protected void consume() throws IOException { + SortedKeyValueIterator<Key,Value> source = getSource(); int count = 0; if (inclusive) - while (getSource().hasTop() && !colFamSet.contains(getSource().getTopKey().getColumnFamilyData())) { + while (source.hasTop() && !colFamSet.contains(source.getTopKey().getColumnFamilyData())) { if (count < 10) { // it is quicker to call next if we are close, but we never know if we are close // so give next a try a few times - getSource().next(); + source.next(); count++; } else { - ByteSequence higherCF = sortedColFams.higher(getSource().getTopKey().getColumnFamilyData()); + ByteSequence higherCF = sortedColFams.higher(source.getTopKey().getColumnFamilyData()); if (higherCF == null) { // seek to the next row - reseek(getSource().getTopKey().followingKey(PartialKey.ROW)); + reseek(source.getTopKey().followingKey(PartialKey.ROW)); } else { // seek to the next column family in the sorted list of column families - reseek(new Key(getSource().getTopKey().getRowData().toArray(), higherCF.toArray(), new byte[0], new byte[0], Long.MAX_VALUE)); + reseek(new Key(source.getTopKey().getRowData().toArray(), higherCF.toArray(), new byte[0], new byte[0], Long.MAX_VALUE)); } count = 0; } } else if (colFamSet != null && colFamSet.size() > 0) - while (getSource().hasTop() && colFamSet.contains(getSource().getTopKey().getColumnFamilyData())) { + while (source.hasTop() && colFamSet.contains(source.getTopKey().getColumnFamilyData())) { if (count < 10) { - getSource().next(); + source.next(); count++; } else { // seek to the next column family in the data - reseek(getSource().getTopKey().followingKey(PartialKey.ROW_COLFAM)); + reseek(source.getTopKey().followingKey(PartialKey.ROW_COLFAM)); count = 0; } } http://git-wip-us.apache.org/repos/asf/accumulo/blob/2ff26780/core/src/main/java/org/apache/accumulo/core/iterators/system/ColumnQualifierFilter.java ---------------------------------------------------------------------- diff --git a/core/src/main/java/org/apache/accumulo/core/iterators/system/ColumnQualifierFilter.java b/core/src/main/java/org/apache/accumulo/core/iterators/system/ColumnQualifierFilter.java index 866f04f..c0a5fcf 100644 --- a/core/src/main/java/org/apache/accumulo/core/iterators/system/ColumnQualifierFilter.java +++ b/core/src/main/java/org/apache/accumulo/core/iterators/system/ColumnQualifierFilter.java @@ -18,7 +18,6 @@ package org.apache.accumulo.core.iterators.system; import java.util.HashMap; import java.util.HashSet; -import java.util.Iterator; import java.util.Set; import org.apache.accumulo.core.data.ArrayByteSequence; @@ -31,18 +30,36 @@ import org.apache.accumulo.core.iterators.IteratorEnvironment; import org.apache.accumulo.core.iterators.SortedKeyValueIterator; public class ColumnQualifierFilter extends Filter { - private boolean scanColumns; + private final boolean scanColumns; private HashSet<ByteSequence> columnFamilies; private HashMap<ByteSequence,HashSet<ByteSequence>> columnsQualifiers; - public ColumnQualifierFilter() {} - public ColumnQualifierFilter(SortedKeyValueIterator<Key,Value> iterator, Set<Column> columns) { setSource(iterator); - init(columns); + this.columnFamilies = new HashSet<>(); + this.columnsQualifiers = new HashMap<>(); + + for (Column col : columns) { + if (col.columnQualifier != null) { + ArrayByteSequence cq = new ArrayByteSequence(col.columnQualifier); + HashSet<ByteSequence> cfset = this.columnsQualifiers.get(cq); + if (cfset == null) { + cfset = new HashSet<>(); + this.columnsQualifiers.put(cq, cfset); + } + + cfset.add(new ArrayByteSequence(col.columnFamily)); + } else { + // this whole column family should pass + columnFamilies.add(new ArrayByteSequence(col.columnFamily)); + } + } + + // only take action when column qualifies are present + scanColumns = this.columnsQualifiers.size() > 0; } - public ColumnQualifierFilter(SortedKeyValueIterator<Key,Value> iterator, HashSet<ByteSequence> columnFamilies, + private ColumnQualifierFilter(SortedKeyValueIterator<Key,Value> iterator, HashSet<ByteSequence> columnFamilies, HashMap<ByteSequence,HashSet<ByteSequence>> columnsQualifiers, boolean scanColumns) { setSource(iterator); this.columnFamilies = columnFamilies; @@ -65,33 +82,24 @@ public class ColumnQualifierFilter extends Filter { return cfset != null && cfset.contains(key.getColumnFamilyData()); } - public void init(Set<Column> columns) { - this.columnFamilies = new HashSet<>(); - this.columnsQualifiers = new HashMap<>(); - - for (Iterator<Column> iter = columns.iterator(); iter.hasNext();) { - Column col = iter.next(); - if (col.columnQualifier != null) { - ArrayByteSequence cq = new ArrayByteSequence(col.columnQualifier); - HashSet<ByteSequence> cfset = this.columnsQualifiers.get(cq); - if (cfset == null) { - cfset = new HashSet<>(); - this.columnsQualifiers.put(cq, cfset); - } + @Override + public SortedKeyValueIterator<Key,Value> deepCopy(IteratorEnvironment env) { + return new ColumnQualifierFilter(getSource().deepCopy(env), columnFamilies, columnsQualifiers, scanColumns); + } - cfset.add(new ArrayByteSequence(col.columnFamily)); - } else { - // this whole column family should pass - columnFamilies.add(new ArrayByteSequence(col.columnFamily)); + public static SortedKeyValueIterator<Key,Value> wrap(SortedKeyValueIterator<Key,Value> source, Set<Column> cols) { + boolean sawNonNullQual = false; + for (Column col : cols) { + if (col.getColumnQualifier() != null) { + sawNonNullQual = true; + break; } } - // only take action when column qualifies are present - scanColumns = this.columnsQualifiers.size() > 0; - } - - @Override - public SortedKeyValueIterator<Key,Value> deepCopy(IteratorEnvironment env) { - return new ColumnQualifierFilter(getSource().deepCopy(env), columnFamilies, columnsQualifiers, scanColumns); + if (sawNonNullQual) { + return new ColumnQualifierFilter(source, cols); + } else { + return source; + } } } http://git-wip-us.apache.org/repos/asf/accumulo/blob/2ff26780/core/src/main/java/org/apache/accumulo/core/iterators/system/DeletingIterator.java ---------------------------------------------------------------------- diff --git a/core/src/main/java/org/apache/accumulo/core/iterators/system/DeletingIterator.java b/core/src/main/java/org/apache/accumulo/core/iterators/system/DeletingIterator.java index abdb6c1..fdceef0 100644 --- a/core/src/main/java/org/apache/accumulo/core/iterators/system/DeletingIterator.java +++ b/core/src/main/java/org/apache/accumulo/core/iterators/system/DeletingIterator.java @@ -53,11 +53,12 @@ public class DeletingIterator extends WrappingIterator { @Override public void next() throws IOException { + SortedKeyValueIterator<Key,Value> source = getSource(); if (super.getTopKey().isDeleted()) - skipRowColumn(); + skipRowColumn(source); else - getSource().next(); - findTop(); + source.next(); + findTop(source); } @Override @@ -66,10 +67,11 @@ public class DeletingIterator extends WrappingIterator { Range seekRange = IteratorUtil.maximizeStartKeyTimeStamp(range); super.seek(seekRange, columnFamilies, inclusive); - findTop(); + SortedKeyValueIterator<Key,Value> source = getSource(); + findTop(source); if (range.getStartKey() != null) { - while (getSource().hasTop() && getSource().getTopKey().compareTo(range.getStartKey(), PartialKey.ROW_COLFAM_COLQUAL_COLVIS_TIME) < 0) { + while (source.hasTop() && source.getTopKey().compareTo(range.getStartKey(), PartialKey.ROW_COLFAM_COLQUAL_COLVIS_TIME) < 0) { next(); } @@ -79,22 +81,22 @@ public class DeletingIterator extends WrappingIterator { } } - private void findTop() throws IOException { + private void findTop(SortedKeyValueIterator<Key,Value> source) throws IOException { if (!propogateDeletes) { - while (getSource().hasTop() && getSource().getTopKey().isDeleted()) { - skipRowColumn(); + while (source.hasTop() && source.getTopKey().isDeleted()) { + skipRowColumn(source); } } } - private void skipRowColumn() throws IOException { - workKey.set(getSource().getTopKey()); + private void skipRowColumn(SortedKeyValueIterator<Key,Value> source) throws IOException { + workKey.set(source.getTopKey()); Key keyToSkip = workKey; - getSource().next(); + source.next(); - while (getSource().hasTop() && getSource().getTopKey().equals(keyToSkip, PartialKey.ROW_COLFAM_COLQUAL_COLVIS)) { - getSource().next(); + while (source.hasTop() && source.getTopKey().equals(keyToSkip, PartialKey.ROW_COLFAM_COLQUAL_COLVIS)) { + source.next(); } } http://git-wip-us.apache.org/repos/asf/accumulo/blob/2ff26780/core/src/main/java/org/apache/accumulo/core/iterators/system/VisibilityFilter.java ---------------------------------------------------------------------- diff --git a/core/src/main/java/org/apache/accumulo/core/iterators/system/VisibilityFilter.java b/core/src/main/java/org/apache/accumulo/core/iterators/system/VisibilityFilter.java index a204ad1..e20938e 100644 --- a/core/src/main/java/org/apache/accumulo/core/iterators/system/VisibilityFilter.java +++ b/core/src/main/java/org/apache/accumulo/core/iterators/system/VisibilityFilter.java @@ -16,6 +16,8 @@ */ package org.apache.accumulo.core.iterators.system; +import org.apache.accumulo.core.data.ArrayByteSequence; +import org.apache.accumulo.core.data.ByteSequence; import org.apache.accumulo.core.data.Key; import org.apache.accumulo.core.data.Value; import org.apache.accumulo.core.iterators.Filter; @@ -26,44 +28,38 @@ import org.apache.accumulo.core.security.ColumnVisibility; import org.apache.accumulo.core.security.VisibilityEvaluator; import org.apache.accumulo.core.security.VisibilityParseException; import org.apache.accumulo.core.util.BadArgumentException; -import org.apache.accumulo.core.util.TextUtil; import org.apache.commons.collections.map.LRUMap; -import org.apache.hadoop.io.Text; import org.slf4j.Logger; import org.slf4j.LoggerFactory; public class VisibilityFilter extends Filter { protected VisibilityEvaluator ve; - protected Text defaultVisibility; + protected ByteSequence defaultVisibility; protected LRUMap cache; - protected Text tmpVis; protected Authorizations authorizations; private static final Logger log = LoggerFactory.getLogger(VisibilityFilter.class); - public VisibilityFilter() {} - - public VisibilityFilter(SortedKeyValueIterator<Key,Value> iterator, Authorizations authorizations, byte[] defaultVisibility) { + private VisibilityFilter(SortedKeyValueIterator<Key,Value> iterator, Authorizations authorizations, byte[] defaultVisibility) { setSource(iterator); this.ve = new VisibilityEvaluator(authorizations); this.authorizations = authorizations; - this.defaultVisibility = new Text(defaultVisibility); + this.defaultVisibility = new ArrayByteSequence(defaultVisibility); this.cache = new LRUMap(1000); - this.tmpVis = new Text(); } @Override public SortedKeyValueIterator<Key,Value> deepCopy(IteratorEnvironment env) { - return new VisibilityFilter(getSource().deepCopy(env), authorizations, TextUtil.getBytes(defaultVisibility)); + return new VisibilityFilter(getSource().deepCopy(env), authorizations, defaultVisibility.toArray()); } @Override public boolean accept(Key k, Value v) { - Text testVis = k.getColumnVisibility(tmpVis); + ByteSequence testVis = k.getColumnVisibilityData(); - if (testVis.getLength() == 0 && defaultVisibility.getLength() == 0) + if (testVis.length() == 0 && defaultVisibility.length() == 0) return true; - else if (testVis.getLength() == 0) + else if (testVis.length() == 0) testVis = defaultVisibility; Boolean b = (Boolean) cache.get(testVis); @@ -71,8 +67,8 @@ public class VisibilityFilter extends Filter { return b; try { - Boolean bb = ve.evaluate(new ColumnVisibility(testVis)); - cache.put(new Text(testVis), bb); + Boolean bb = ve.evaluate(new ColumnVisibility(testVis.toArray())); + cache.put(testVis, bb); return bb; } catch (VisibilityParseException e) { log.error("Parse Error", e); @@ -82,4 +78,29 @@ public class VisibilityFilter extends Filter { return false; } } + + private static class EmptyAuthsVisibilityFilter extends Filter { + + public EmptyAuthsVisibilityFilter(SortedKeyValueIterator<Key,Value> source) { + setSource(source); + } + + @Override + public SortedKeyValueIterator<Key,Value> deepCopy(IteratorEnvironment env) { + return new EmptyAuthsVisibilityFilter(getSource().deepCopy(env)); + } + + @Override + public boolean accept(Key k, Value v) { + return k.getColumnVisibilityData().length() == 0; + } + } + + public static SortedKeyValueIterator<Key,Value> wrap(SortedKeyValueIterator<Key,Value> source, Authorizations authorizations, byte[] defaultVisibility) { + if (authorizations.isEmpty() && defaultVisibility.length == 0) { + return new EmptyAuthsVisibilityFilter(source); + } else { + return new VisibilityFilter(source, authorizations, defaultVisibility); + } + } } http://git-wip-us.apache.org/repos/asf/accumulo/blob/2ff26780/core/src/main/java/org/apache/accumulo/core/iterators/user/VersioningIterator.java ---------------------------------------------------------------------- diff --git a/core/src/main/java/org/apache/accumulo/core/iterators/user/VersioningIterator.java b/core/src/main/java/org/apache/accumulo/core/iterators/user/VersioningIterator.java index 88ba20d..3334d9f 100644 --- a/core/src/main/java/org/apache/accumulo/core/iterators/user/VersioningIterator.java +++ b/core/src/main/java/org/apache/accumulo/core/iterators/user/VersioningIterator.java @@ -97,11 +97,12 @@ public class VersioningIterator extends WrappingIterator implements OptionDescri super.next(); int count = 0; - while (getSource().hasTop() && getSource().getTopKey().equals(keyToSkip, PartialKey.ROW_COLFAM_COLQUAL_COLVIS)) { + SortedKeyValueIterator<Key,Value> source = getSource(); + while (source.hasTop() && source.getTopKey().equals(keyToSkip, PartialKey.ROW_COLFAM_COLQUAL_COLVIS)) { if (count < maxCount) { // it is quicker to call next if we are close, but we never know if we are close // so give next a try a few times - getSource().next(); + source.next(); count++; } else { reseek(keyToSkip.followingKey(PartialKey.ROW_COLFAM_COLQUAL_COLVIS)); http://git-wip-us.apache.org/repos/asf/accumulo/blob/2ff26780/core/src/main/java/org/apache/accumulo/core/iterators/user/VisibilityFilter.java ---------------------------------------------------------------------- diff --git a/core/src/main/java/org/apache/accumulo/core/iterators/user/VisibilityFilter.java b/core/src/main/java/org/apache/accumulo/core/iterators/user/VisibilityFilter.java index 6e55aec..1f75a27 100644 --- a/core/src/main/java/org/apache/accumulo/core/iterators/user/VisibilityFilter.java +++ b/core/src/main/java/org/apache/accumulo/core/iterators/user/VisibilityFilter.java @@ -22,23 +22,30 @@ import java.io.IOException; import java.util.Map; import org.apache.accumulo.core.client.IteratorSetting; +import org.apache.accumulo.core.data.ByteSequence; import org.apache.accumulo.core.data.Key; import org.apache.accumulo.core.data.Value; +import org.apache.accumulo.core.iterators.Filter; import org.apache.accumulo.core.iterators.IteratorEnvironment; import org.apache.accumulo.core.iterators.OptionDescriber; import org.apache.accumulo.core.iterators.SortedKeyValueIterator; import org.apache.accumulo.core.security.Authorizations; import org.apache.accumulo.core.security.ColumnVisibility; import org.apache.accumulo.core.security.VisibilityEvaluator; +import org.apache.accumulo.core.security.VisibilityParseException; import org.apache.accumulo.core.util.BadArgumentException; import org.apache.commons.collections.map.LRUMap; -import org.apache.hadoop.io.Text; +import org.slf4j.Logger; +import org.slf4j.LoggerFactory; /** * */ -public class VisibilityFilter extends org.apache.accumulo.core.iterators.system.VisibilityFilter implements OptionDescriber { +public class VisibilityFilter extends Filter implements OptionDescriber { + protected VisibilityEvaluator ve; + protected LRUMap cache; + private static final Logger log = LoggerFactory.getLogger(VisibilityFilter.class); private static final String AUTHS = "auths"; private static final String FILTER_INVALID_ONLY = "filterInvalid"; @@ -47,9 +54,7 @@ public class VisibilityFilter extends org.apache.accumulo.core.iterators.system. /** * */ - public VisibilityFilter() { - super(); - } + public VisibilityFilter() {} @Override public void init(SortedKeyValueIterator<Key,Value> source, Map<String,String> options, IteratorEnvironment env) throws IOException { @@ -61,29 +66,43 @@ public class VisibilityFilter extends org.apache.accumulo.core.iterators.system. String auths = options.get(AUTHS); Authorizations authObj = auths == null || auths.isEmpty() ? new Authorizations() : new Authorizations(auths.getBytes(UTF_8)); this.ve = new VisibilityEvaluator(authObj); - this.defaultVisibility = new Text(); } this.cache = new LRUMap(1000); - this.tmpVis = new Text(); } @Override public boolean accept(Key k, Value v) { + ByteSequence testVis = k.getColumnVisibilityData(); if (filterInvalid) { - Text testVis = k.getColumnVisibility(tmpVis); Boolean b = (Boolean) cache.get(testVis); if (b != null) return b; try { - new ColumnVisibility(testVis); - cache.put(new Text(testVis), true); + new ColumnVisibility(testVis.toArray()); + cache.put(testVis, true); return true; } catch (BadArgumentException e) { - cache.put(new Text(testVis), false); + cache.put(testVis, false); return false; } } else { - return super.accept(k, v); + if (testVis.length() == 0) { + return true; + } + + Boolean b = (Boolean) cache.get(testVis); + if (b != null) + return b; + + try { + Boolean bb = ve.evaluate(new ColumnVisibility(testVis.toArray())); + cache.put(testVis, bb); + return bb; + } catch (VisibilityParseException | BadArgumentException e) { + log.error("Parse Error", e); + return false; + } + } } http://git-wip-us.apache.org/repos/asf/accumulo/blob/2ff26780/core/src/test/java/org/apache/accumulo/core/iterators/system/ColumnFilterTest.java ---------------------------------------------------------------------- diff --git a/core/src/test/java/org/apache/accumulo/core/iterators/system/ColumnFilterTest.java b/core/src/test/java/org/apache/accumulo/core/iterators/system/ColumnFilterTest.java index 6158476..908eae1 100644 --- a/core/src/test/java/org/apache/accumulo/core/iterators/system/ColumnFilterTest.java +++ b/core/src/test/java/org/apache/accumulo/core/iterators/system/ColumnFilterTest.java @@ -16,14 +16,21 @@ */ package org.apache.accumulo.core.iterators.system; +import java.util.Collections; import java.util.HashSet; +import java.util.TreeMap; import junit.framework.TestCase; +import org.apache.accumulo.core.data.ByteSequence; import org.apache.accumulo.core.data.Column; import org.apache.accumulo.core.data.Key; +import org.apache.accumulo.core.data.Range; import org.apache.accumulo.core.data.Value; +import org.apache.accumulo.core.iterators.SortedKeyValueIterator; +import org.apache.accumulo.core.iterators.SortedMapIterator; import org.apache.hadoop.io.Text; +import org.junit.Assert; public class ColumnFilterTest extends TestCase { @@ -40,39 +47,60 @@ public class ColumnFilterTest extends TestCase { } public void test1() { - HashSet<Column> columns = new HashSet<>(); + TreeMap<Key,Value> data = new TreeMap<Key,Value>(); + data.put(newKey("r1", "cf1", "cq1"), new Value()); + data.put(newKey("r1", "cf2", "cq1"), new Value()); + HashSet<Column> columns = new HashSet<>(); columns.add(newColumn("cf1")); - ColumnQualifierFilter cf = new ColumnQualifierFilter(null, columns); - - assertTrue(cf.accept(newKey("r1", "cf1", "cq1"), new Value(new byte[0]))); - assertTrue(cf.accept(newKey("r1", "cf2", "cq1"), new Value(new byte[0]))); + SortedMapIterator smi = new SortedMapIterator(data); + SortedKeyValueIterator<Key,Value> cf = ColumnQualifierFilter.wrap(smi, columns); + Assert.assertSame(smi, cf); } - public void test2() { + public void test2() throws Exception { + + TreeMap<Key,Value> data = new TreeMap<Key,Value>(); + data.put(newKey("r1", "cf1", "cq1"), new Value()); + data.put(newKey("r1", "cf2", "cq1"), new Value()); + data.put(newKey("r1", "cf2", "cq2"), new Value()); + HashSet<Column> columns = new HashSet<>(); columns.add(newColumn("cf1")); columns.add(newColumn("cf2", "cq1")); - ColumnQualifierFilter cf = new ColumnQualifierFilter(null, columns); + SortedKeyValueIterator<Key,Value> cf = ColumnQualifierFilter.wrap(new SortedMapIterator(data), columns); + cf.seek(new Range(), Collections.<ByteSequence> emptySet(), false); - assertTrue(cf.accept(newKey("r1", "cf1", "cq1"), new Value(new byte[0]))); - assertTrue(cf.accept(newKey("r1", "cf2", "cq1"), new Value(new byte[0]))); - assertFalse(cf.accept(newKey("r1", "cf2", "cq2"), new Value(new byte[0]))); + Assert.assertTrue(cf.hasTop()); + Assert.assertEquals(newKey("r1", "cf1", "cq1"), cf.getTopKey()); + cf.next(); + Assert.assertTrue(cf.hasTop()); + Assert.assertEquals(newKey("r1", "cf2", "cq1"), cf.getTopKey()); + cf.next(); + Assert.assertFalse(cf.hasTop()); } - public void test3() { + public void test3() throws Exception { + + TreeMap<Key,Value> data = new TreeMap<Key,Value>(); + data.put(newKey("r1", "cf1", "cq1"), new Value()); + data.put(newKey("r1", "cf2", "cq1"), new Value()); + data.put(newKey("r1", "cf2", "cq2"), new Value()); + HashSet<Column> columns = new HashSet<>(); columns.add(newColumn("cf2", "cq1")); - ColumnQualifierFilter cf = new ColumnQualifierFilter(null, columns); + SortedKeyValueIterator<Key,Value> cf = ColumnQualifierFilter.wrap(new SortedMapIterator(data), columns); + cf.seek(new Range(), Collections.<ByteSequence> emptySet(), false); - assertFalse(cf.accept(newKey("r1", "cf1", "cq1"), new Value(new byte[0]))); - assertTrue(cf.accept(newKey("r1", "cf2", "cq1"), new Value(new byte[0]))); - assertFalse(cf.accept(newKey("r1", "cf2", "cq2"), new Value(new byte[0]))); + Assert.assertTrue(cf.hasTop()); + Assert.assertEquals(newKey("r1", "cf2", "cq1"), cf.getTopKey()); + cf.next(); + Assert.assertFalse(cf.hasTop()); } } http://git-wip-us.apache.org/repos/asf/accumulo/blob/2ff26780/core/src/test/java/org/apache/accumulo/core/iterators/system/VisibilityFilterTest.java ---------------------------------------------------------------------- diff --git a/core/src/test/java/org/apache/accumulo/core/iterators/system/VisibilityFilterTest.java b/core/src/test/java/org/apache/accumulo/core/iterators/system/VisibilityFilterTest.java index 68323c6..2e5ca04 100644 --- a/core/src/test/java/org/apache/accumulo/core/iterators/system/VisibilityFilterTest.java +++ b/core/src/test/java/org/apache/accumulo/core/iterators/system/VisibilityFilterTest.java @@ -26,6 +26,7 @@ import org.apache.accumulo.core.data.ByteSequence; import org.apache.accumulo.core.data.Key; import org.apache.accumulo.core.data.Range; import org.apache.accumulo.core.data.Value; +import org.apache.accumulo.core.iterators.SortedKeyValueIterator; import org.apache.accumulo.core.iterators.SortedMapIterator; import org.apache.accumulo.core.security.Authorizations; import org.apache.log4j.Level; @@ -37,7 +38,7 @@ public class VisibilityFilterTest extends TestCase { TreeMap<Key,Value> tm = new TreeMap<>(); tm.put(new Key("r1", "cf1", "cq1", "A&"), new Value(new byte[0])); - VisibilityFilter filter = new VisibilityFilter(new SortedMapIterator(tm), new Authorizations("A"), "".getBytes()); + SortedKeyValueIterator<Key,Value> filter = VisibilityFilter.wrap(new SortedMapIterator(tm), new Authorizations("A"), "".getBytes()); // suppress logging Level prevLevel = Logger.getLogger(VisibilityFilter.class).getLevel(); @@ -49,4 +50,21 @@ public class VisibilityFilterTest extends TestCase { Logger.getLogger(VisibilityFilter.class).setLevel(prevLevel); } + public void testEmptyAuths() throws IOException { + TreeMap<Key,Value> tm = new TreeMap<>(); + + tm.put(new Key("r1", "cf1", "cq1", ""), new Value(new byte[0])); + tm.put(new Key("r1", "cf1", "cq2", "C"), new Value(new byte[0])); + tm.put(new Key("r1", "cf1", "cq3", ""), new Value(new byte[0])); + SortedKeyValueIterator<Key,Value> filter = VisibilityFilter.wrap(new SortedMapIterator(tm), Authorizations.EMPTY, "".getBytes()); + + filter.seek(new Range(), new HashSet<ByteSequence>(), false); + assertTrue(filter.hasTop()); + assertEquals(new Key("r1", "cf1", "cq1", ""), filter.getTopKey()); + filter.next(); + assertTrue(filter.hasTop()); + assertEquals(new Key("r1", "cf1", "cq3", ""), filter.getTopKey()); + filter.next(); + assertFalse(filter.hasTop()); + } } http://git-wip-us.apache.org/repos/asf/accumulo/blob/2ff26780/core/src/test/java/org/apache/accumulo/core/iterators/user/FilterTest.java ---------------------------------------------------------------------- diff --git a/core/src/test/java/org/apache/accumulo/core/iterators/user/FilterTest.java b/core/src/test/java/org/apache/accumulo/core/iterators/user/FilterTest.java index 61d98e6..7d7011b 100644 --- a/core/src/test/java/org/apache/accumulo/core/iterators/user/FilterTest.java +++ b/core/src/test/java/org/apache/accumulo/core/iterators/user/FilterTest.java @@ -277,19 +277,19 @@ public class FilterTest { } assertTrue(tm.size() == 1000); - ColumnQualifierFilter a = new ColumnQualifierFilter(new SortedMapIterator(tm), hsc); + SortedKeyValueIterator<Key,Value> a = ColumnQualifierFilter.wrap(new SortedMapIterator(tm), hsc); a.seek(new Range(), EMPTY_COL_FAMS, false); assertEquals(size(a), 1000); hsc = new HashSet<>(); hsc.add(new Column("a".getBytes(), "b".getBytes(), null)); - a = new ColumnQualifierFilter(new SortedMapIterator(tm), hsc); + a = ColumnQualifierFilter.wrap(new SortedMapIterator(tm), hsc); a.seek(new Range(), EMPTY_COL_FAMS, false); int size = size(a); assertTrue("size was " + size, size == 500); hsc = new HashSet<>(); - a = new ColumnQualifierFilter(new SortedMapIterator(tm), hsc); + a = ColumnQualifierFilter.wrap(new SortedMapIterator(tm), hsc); a.seek(new Range(), EMPTY_COL_FAMS, false); size = size(a); assertTrue("size was " + size, size == 1000); @@ -313,20 +313,20 @@ public class FilterTest { } assertTrue(tm.size() == 1000); - VisibilityFilter a = new VisibilityFilter(new SortedMapIterator(tm), auths, le2.getExpression()); + SortedKeyValueIterator<Key,Value> a = VisibilityFilter.wrap(new SortedMapIterator(tm), auths, le2.getExpression()); a.seek(new Range(), EMPTY_COL_FAMS, false); int size = size(a); assertTrue("size was " + size, size == 750); } - private ColumnQualifierFilter ncqf(TreeMap<Key,Value> tm, Column... columns) throws IOException { + private SortedKeyValueIterator<Key,Value> ncqf(TreeMap<Key,Value> tm, Column... columns) throws IOException { HashSet<Column> hsc = new HashSet<>(); for (Column column : columns) { hsc.add(column); } - ColumnQualifierFilter a = new ColumnQualifierFilter(new SortedMapIterator(tm), hsc); + SortedKeyValueIterator<Key,Value> a = ColumnQualifierFilter.wrap(new SortedMapIterator(tm), hsc); a.seek(new Range(), EMPTY_COL_FAMS, false); return a; } http://git-wip-us.apache.org/repos/asf/accumulo/blob/2ff26780/server/base/src/main/java/org/apache/accumulo/server/problems/ProblemReportingIterator.java ---------------------------------------------------------------------- diff --git a/server/base/src/main/java/org/apache/accumulo/server/problems/ProblemReportingIterator.java b/server/base/src/main/java/org/apache/accumulo/server/problems/ProblemReportingIterator.java index 349ed20..e419780 100644 --- a/server/base/src/main/java/org/apache/accumulo/server/problems/ProblemReportingIterator.java +++ b/server/base/src/main/java/org/apache/accumulo/server/problems/ProblemReportingIterator.java @@ -31,9 +31,9 @@ import org.apache.accumulo.core.iterators.system.InterruptibleIterator; import org.apache.accumulo.server.AccumuloServerContext; public class ProblemReportingIterator implements InterruptibleIterator { - private SortedKeyValueIterator<Key,Value> source; + private final SortedKeyValueIterator<Key,Value> source; private boolean sawError = false; - private boolean continueOnError; + private final boolean continueOnError; private String resource; private String table; private final AccumuloServerContext context; http://git-wip-us.apache.org/repos/asf/accumulo/blob/2ff26780/server/tserver/src/main/java/org/apache/accumulo/tserver/InMemoryMap.java ---------------------------------------------------------------------- diff --git a/server/tserver/src/main/java/org/apache/accumulo/tserver/InMemoryMap.java b/server/tserver/src/main/java/org/apache/accumulo/tserver/InMemoryMap.java index 80eca3c..890b4e0 100644 --- a/server/tserver/src/main/java/org/apache/accumulo/tserver/InMemoryMap.java +++ b/server/tserver/src/main/java/org/apache/accumulo/tserver/InMemoryMap.java @@ -521,13 +521,6 @@ public class InMemoryMap { private SourceSwitchingIterator ssi; private MemoryDataSource mds; - @Override - protected SortedKeyValueIterator<Key,Value> getSource() { - if (closed.get()) - throw new IllegalStateException("Memory iterator is closed"); - return super.getSource(); - } - private MemoryIterator(InterruptibleIterator source) { this(source, new AtomicBoolean(false)); } http://git-wip-us.apache.org/repos/asf/accumulo/blob/2ff26780/server/tserver/src/main/java/org/apache/accumulo/tserver/tablet/ScanDataSource.java ---------------------------------------------------------------------- diff --git a/server/tserver/src/main/java/org/apache/accumulo/tserver/tablet/ScanDataSource.java b/server/tserver/src/main/java/org/apache/accumulo/tserver/tablet/ScanDataSource.java index 5632960..9ef1a7c 100644 --- a/server/tserver/src/main/java/org/apache/accumulo/tserver/tablet/ScanDataSource.java +++ b/server/tserver/src/main/java/org/apache/accumulo/tserver/tablet/ScanDataSource.java @@ -185,7 +185,7 @@ class ScanDataSource implements DataSource { ColumnQualifierFilter colFilter = new ColumnQualifierFilter(cfsi, options.getColumnSet()); - VisibilityFilter visFilter = new VisibilityFilter(colFilter, options.getAuthorizations(), options.getDefaultLabels()); + SortedKeyValueIterator<Key,Value> visFilter = VisibilityFilter.wrap(colFilter, options.getAuthorizations(), options.getDefaultLabels()); if (!loadIters) { return visFilter; http://git-wip-us.apache.org/repos/asf/accumulo/blob/2ff26780/test/src/main/java/org/apache/accumulo/test/performance/scan/CollectTabletStats.java ---------------------------------------------------------------------- diff --git a/test/src/main/java/org/apache/accumulo/test/performance/scan/CollectTabletStats.java b/test/src/main/java/org/apache/accumulo/test/performance/scan/CollectTabletStats.java index 18e0bff..550297b 100644 --- a/test/src/main/java/org/apache/accumulo/test/performance/scan/CollectTabletStats.java +++ b/test/src/main/java/org/apache/accumulo/test/performance/scan/CollectTabletStats.java @@ -431,8 +431,8 @@ public class CollectTabletStats { MultiIterator multiIter = new MultiIterator(iters, ke); DeletingIterator delIter = new DeletingIterator(multiIter, false); ColumnFamilySkippingIterator cfsi = new ColumnFamilySkippingIterator(delIter); - ColumnQualifierFilter colFilter = new ColumnQualifierFilter(cfsi, columnSet); - VisibilityFilter visFilter = new VisibilityFilter(colFilter, authorizations, defaultLabels); + SortedKeyValueIterator<Key,Value> colFilter = ColumnQualifierFilter.wrap(cfsi, columnSet); + SortedKeyValueIterator<Key,Value> visFilter = VisibilityFilter.wrap(colFilter, authorizations, defaultLabels); if (useTableIterators) return IteratorUtil.loadIterators(IteratorScope.scan, visFilter, ke, conf, ssiList, ssio, null);