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 469BB200B8C for ; Tue, 23 Aug 2016 00:07:14 +0200 (CEST) Received: by cust-asf.ponee.io (Postfix) id 43665160ABC; Mon, 22 Aug 2016 22:07:14 +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 EE0F7160AC4 for ; Tue, 23 Aug 2016 00:07:11 +0200 (CEST) Received: (qmail 18280 invoked by uid 500); 22 Aug 2016 22:07:10 -0000 Mailing-List: contact commits-help@accumulo.apache.org; run by ezmlm Precedence: bulk List-Help: List-Unsubscribe: List-Post: List-Id: Reply-To: dev@accumulo.apache.org Delivered-To: mailing list commits@accumulo.apache.org Received: (qmail 18169 invoked by uid 99); 22 Aug 2016 22:07:10 -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, 22 Aug 2016 22:07:10 +0000 Received: by git1-us-west.apache.org (ASF Mail Server at git1-us-west.apache.org, from userid 33) id 7C7C3E020A; Mon, 22 Aug 2016 22:07:10 +0000 (UTC) Content-Type: text/plain; charset="us-ascii" MIME-Version: 1.0 Content-Transfer-Encoding: 7bit From: ctubbsii@apache.org To: commits@accumulo.apache.org Date: Mon, 22 Aug 2016 22:07:14 -0000 Message-Id: In-Reply-To: <52a85587a9e44dc784d5b7249ccfef92@git.apache.org> References: <52a85587a9e44dc784d5b7249ccfef92@git.apache.org> X-Mailer: ASF-Git Admin Mailer Subject: [5/8] accumulo git commit: Revert "ACCUMULO-2589 Drop deprecated code from 1.x" archived-at: Mon, 22 Aug 2016 22:07:14 -0000 http://git-wip-us.apache.org/repos/asf/accumulo/blob/4f2e6472/core/src/main/java/org/apache/accumulo/core/client/mock/MockTableOperations.java ---------------------------------------------------------------------- diff --git a/core/src/main/java/org/apache/accumulo/core/client/mock/MockTableOperations.java b/core/src/main/java/org/apache/accumulo/core/client/mock/MockTableOperations.java new file mode 100644 index 0000000..de89137 --- /dev/null +++ b/core/src/main/java/org/apache/accumulo/core/client/mock/MockTableOperations.java @@ -0,0 +1,505 @@ +/* + * 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.accumulo.core.client.mock; + +import java.io.DataInputStream; +import java.io.IOException; +import java.util.ArrayList; +import java.util.Collection; +import java.util.Collections; +import java.util.HashMap; +import java.util.HashSet; +import java.util.List; +import java.util.Map; +import java.util.Map.Entry; +import java.util.Set; +import java.util.SortedSet; +import java.util.TreeSet; + +import org.apache.accumulo.core.client.AccumuloException; +import org.apache.accumulo.core.client.AccumuloSecurityException; +import org.apache.accumulo.core.client.IteratorSetting; +import org.apache.accumulo.core.client.NamespaceNotFoundException; +import org.apache.accumulo.core.client.TableExistsException; +import org.apache.accumulo.core.client.TableNotFoundException; +import org.apache.accumulo.core.client.admin.CompactionConfig; +import org.apache.accumulo.core.client.admin.DiskUsage; +import org.apache.accumulo.core.client.admin.FindMax; +import org.apache.accumulo.core.client.admin.Locations; +import org.apache.accumulo.core.client.admin.NewTableConfiguration; +import org.apache.accumulo.core.client.admin.TimeType; +import org.apache.accumulo.core.client.impl.TableOperationsHelper; +import org.apache.accumulo.core.client.impl.Tables; +import org.apache.accumulo.core.client.sample.SamplerConfiguration; +import org.apache.accumulo.core.conf.AccumuloConfiguration; +import org.apache.accumulo.core.data.Key; +import org.apache.accumulo.core.data.Mutation; +import org.apache.accumulo.core.data.Range; +import org.apache.accumulo.core.data.Value; +import org.apache.accumulo.core.file.FileOperations; +import org.apache.accumulo.core.file.FileSKVIterator; +import org.apache.accumulo.core.metadata.MetadataTable; +import org.apache.accumulo.core.metadata.RootTable; +import org.apache.accumulo.core.security.Authorizations; +import org.apache.accumulo.core.security.ColumnVisibility; +import org.apache.accumulo.start.classloader.vfs.AccumuloVFSClassLoader; +import org.apache.commons.lang.NotImplementedException; +import org.apache.hadoop.fs.FSDataOutputStream; +import org.apache.hadoop.fs.FileStatus; +import org.apache.hadoop.fs.FileSystem; +import org.apache.hadoop.fs.Path; +import org.apache.hadoop.io.Text; +import org.slf4j.Logger; +import org.slf4j.LoggerFactory; +import static com.google.common.base.Preconditions.checkArgument; + +/** + * @deprecated since 1.8.0; use MiniAccumuloCluster or a standard mock framework instead. + */ +@Deprecated +class MockTableOperations extends TableOperationsHelper { + private static final Logger log = LoggerFactory.getLogger(MockTableOperations.class); + private static final byte[] ZERO = {0}; + private final MockAccumulo acu; + private final String username; + + MockTableOperations(MockAccumulo acu, String username) { + this.acu = acu; + this.username = username; + } + + @Override + public SortedSet list() { + return new TreeSet<>(acu.tables.keySet()); + } + + @Override + public boolean exists(String tableName) { + return acu.tables.containsKey(tableName); + } + + private boolean namespaceExists(String namespace) { + return acu.namespaces.containsKey(namespace); + } + + @Override + public void create(String tableName) throws AccumuloException, AccumuloSecurityException, TableExistsException { + create(tableName, new NewTableConfiguration()); + } + + @Override + @Deprecated + public void create(String tableName, boolean versioningIter) throws AccumuloException, AccumuloSecurityException, TableExistsException { + create(tableName, versioningIter, TimeType.MILLIS); + } + + @Override + @Deprecated + public void create(String tableName, boolean versioningIter, TimeType timeType) throws AccumuloException, AccumuloSecurityException, TableExistsException { + NewTableConfiguration ntc = new NewTableConfiguration().setTimeType(timeType); + + if (versioningIter) + create(tableName, ntc); + else + create(tableName, ntc.withoutDefaultIterators()); + } + + @Override + public void create(String tableName, NewTableConfiguration ntc) throws AccumuloException, AccumuloSecurityException, TableExistsException { + String namespace = Tables.qualify(tableName).getFirst(); + + checkArgument(tableName.matches(Tables.VALID_NAME_REGEX)); + if (exists(tableName)) + throw new TableExistsException(tableName, tableName, ""); + checkArgument(namespaceExists(namespace), "Namespace (" + namespace + ") does not exist, create it first"); + acu.createTable(username, tableName, ntc.getTimeType(), ntc.getProperties()); + } + + @Override + public void addSplits(String tableName, SortedSet partitionKeys) throws TableNotFoundException, AccumuloException, AccumuloSecurityException { + if (!exists(tableName)) + throw new TableNotFoundException(tableName, tableName, ""); + acu.addSplits(tableName, partitionKeys); + } + + @Deprecated + @Override + public Collection getSplits(String tableName) throws TableNotFoundException { + return listSplits(tableName); + } + + @Deprecated + @Override + public Collection getSplits(String tableName, int maxSplits) throws TableNotFoundException { + return listSplits(tableName); + } + + @Override + public Collection listSplits(String tableName) throws TableNotFoundException { + if (!exists(tableName)) + throw new TableNotFoundException(tableName, tableName, ""); + return acu.getSplits(tableName); + } + + @Override + public Collection listSplits(String tableName, int maxSplits) throws TableNotFoundException { + return listSplits(tableName); + } + + @Override + public void delete(String tableName) throws AccumuloException, AccumuloSecurityException, TableNotFoundException { + if (!exists(tableName)) + throw new TableNotFoundException(tableName, tableName, ""); + acu.tables.remove(tableName); + } + + @Override + public void rename(String oldTableName, String newTableName) throws AccumuloSecurityException, TableNotFoundException, AccumuloException, + TableExistsException { + if (!exists(oldTableName)) + throw new TableNotFoundException(oldTableName, oldTableName, ""); + if (exists(newTableName)) + throw new TableExistsException(newTableName, newTableName, ""); + MockTable t = acu.tables.remove(oldTableName); + String namespace = Tables.qualify(newTableName).getFirst(); + MockNamespace n = acu.namespaces.get(namespace); + if (n == null) { + n = new MockNamespace(); + } + t.setNamespaceName(namespace); + t.setNamespace(n); + acu.namespaces.put(namespace, n); + acu.tables.put(newTableName, t); + } + + @Deprecated + @Override + public void flush(String tableName) throws AccumuloException, AccumuloSecurityException {} + + @Override + public void setProperty(String tableName, String property, String value) throws AccumuloException, AccumuloSecurityException { + acu.tables.get(tableName).settings.put(property, value); + } + + @Override + public void removeProperty(String tableName, String property) throws AccumuloException, AccumuloSecurityException { + acu.tables.get(tableName).settings.remove(property); + } + + @Override + public Iterable> getProperties(String tableName) throws TableNotFoundException { + String namespace = Tables.qualify(tableName).getFirst(); + if (!exists(tableName)) { + if (!namespaceExists(namespace)) + throw new TableNotFoundException(tableName, new NamespaceNotFoundException(null, namespace, null)); + throw new TableNotFoundException(null, tableName, null); + } + + Set> props = new HashSet<>(acu.namespaces.get(namespace).settings.entrySet()); + + Set> tableProps = acu.tables.get(tableName).settings.entrySet(); + for (Entry e : tableProps) { + if (props.contains(e)) { + props.remove(e); + } + props.add(e); + } + return props; + } + + @Override + public void setLocalityGroups(String tableName, Map> groups) throws AccumuloException, AccumuloSecurityException, TableNotFoundException { + if (!exists(tableName)) + throw new TableNotFoundException(tableName, tableName, ""); + acu.tables.get(tableName).setLocalityGroups(groups); + } + + @Override + public Map> getLocalityGroups(String tableName) throws AccumuloException, TableNotFoundException { + if (!exists(tableName)) + throw new TableNotFoundException(tableName, tableName, ""); + return acu.tables.get(tableName).getLocalityGroups(); + } + + @Override + public Set splitRangeByTablets(String tableName, Range range, int maxSplits) throws AccumuloException, AccumuloSecurityException, + TableNotFoundException { + if (!exists(tableName)) + throw new TableNotFoundException(tableName, tableName, ""); + return Collections.singleton(range); + } + + @Override + public void importDirectory(String tableName, String dir, String failureDir, boolean setTime) throws IOException, AccumuloException, + AccumuloSecurityException, TableNotFoundException { + long time = System.currentTimeMillis(); + MockTable table = acu.tables.get(tableName); + if (table == null) { + throw new TableNotFoundException(null, tableName, "The table was not found"); + } + Path importPath = new Path(dir); + Path failurePath = new Path(failureDir); + + FileSystem fs = acu.getFileSystem(); + /* + * check preconditions + */ + // directories are directories + if (fs.isFile(importPath)) { + throw new IOException("Import path must be a directory."); + } + if (fs.isFile(failurePath)) { + throw new IOException("Failure path must be a directory."); + } + // failures are writable + Path createPath = failurePath.suffix("/.createFile"); + FSDataOutputStream createStream = null; + try { + createStream = fs.create(createPath); + } catch (IOException e) { + throw new IOException("Error path is not writable."); + } finally { + if (createStream != null) { + createStream.close(); + } + } + fs.delete(createPath, false); + // failures are empty + FileStatus[] failureChildStats = fs.listStatus(failurePath); + if (failureChildStats.length > 0) { + throw new IOException("Error path must be empty."); + } + /* + * Begin the import - iterate the files in the path + */ + for (FileStatus importStatus : fs.listStatus(importPath)) { + try { + FileSKVIterator importIterator = FileOperations.getInstance().newReaderBuilder().forFile(importStatus.getPath().toString(), fs, fs.getConf()) + .withTableConfiguration(AccumuloConfiguration.getDefaultConfiguration()).seekToBeginning().build(); + while (importIterator.hasTop()) { + Key key = importIterator.getTopKey(); + Value value = importIterator.getTopValue(); + if (setTime) { + key.setTimestamp(time); + } + Mutation mutation = new Mutation(key.getRow()); + if (!key.isDeleted()) { + mutation.put(key.getColumnFamily(), key.getColumnQualifier(), new ColumnVisibility(key.getColumnVisibilityData().toArray()), key.getTimestamp(), + value); + } else { + mutation.putDelete(key.getColumnFamily(), key.getColumnQualifier(), new ColumnVisibility(key.getColumnVisibilityData().toArray()), + key.getTimestamp()); + } + table.addMutation(mutation); + importIterator.next(); + } + } catch (Exception e) { + FSDataOutputStream failureWriter = null; + DataInputStream failureReader = null; + try { + failureWriter = fs.create(failurePath.suffix("/" + importStatus.getPath().getName())); + failureReader = fs.open(importStatus.getPath()); + int read = 0; + byte[] buffer = new byte[1024]; + while (-1 != (read = failureReader.read(buffer))) { + failureWriter.write(buffer, 0, read); + } + } finally { + if (failureReader != null) + failureReader.close(); + if (failureWriter != null) + failureWriter.close(); + } + } + fs.delete(importStatus.getPath(), true); + } + } + + @Override + public void offline(String tableName) throws AccumuloSecurityException, AccumuloException, TableNotFoundException { + offline(tableName, false); + } + + @Override + public void offline(String tableName, boolean wait) throws AccumuloSecurityException, AccumuloException, TableNotFoundException { + if (!exists(tableName)) + throw new TableNotFoundException(tableName, tableName, ""); + } + + @Override + public void online(String tableName) throws AccumuloSecurityException, AccumuloException, TableNotFoundException { + online(tableName, false); + } + + @Override + public void online(String tableName, boolean wait) throws AccumuloSecurityException, AccumuloException, TableNotFoundException { + if (!exists(tableName)) + throw new TableNotFoundException(tableName, tableName, ""); + } + + @Override + public void clearLocatorCache(String tableName) throws TableNotFoundException { + if (!exists(tableName)) + throw new TableNotFoundException(tableName, tableName, ""); + } + + @Override + public Map tableIdMap() { + Map result = new HashMap<>(); + for (Entry entry : acu.tables.entrySet()) { + String table = entry.getKey(); + if (RootTable.NAME.equals(table)) + result.put(table, RootTable.ID); + else if (MetadataTable.NAME.equals(table)) + result.put(table, MetadataTable.ID); + else + result.put(table, entry.getValue().getTableId()); + } + return result; + } + + @Override + public List getDiskUsage(Set tables) throws AccumuloException, AccumuloSecurityException { + + List diskUsages = new ArrayList<>(); + diskUsages.add(new DiskUsage(new TreeSet<>(tables), 0l)); + + return diskUsages; + } + + @Override + public void merge(String tableName, Text start, Text end) throws AccumuloException, AccumuloSecurityException, TableNotFoundException { + if (!exists(tableName)) + throw new TableNotFoundException(tableName, tableName, ""); + acu.merge(tableName, start, end); + } + + @Override + public void deleteRows(String tableName, Text start, Text end) throws AccumuloException, AccumuloSecurityException, TableNotFoundException { + if (!exists(tableName)) + throw new TableNotFoundException(tableName, tableName, ""); + MockTable t = acu.tables.get(tableName); + Text startText = start != null ? new Text(start) : new Text(); + if (startText.getLength() == 0 && end == null) { + t.table.clear(); + return; + } + Text endText = end != null ? new Text(end) : new Text(t.table.lastKey().getRow().getBytes()); + startText.append(ZERO, 0, 1); + endText.append(ZERO, 0, 1); + Set keep = new TreeSet<>(t.table.subMap(new Key(startText), new Key(endText)).keySet()); + t.table.keySet().removeAll(keep); + } + + @Override + public void compact(String tableName, Text start, Text end, boolean flush, boolean wait) throws AccumuloSecurityException, TableNotFoundException, + AccumuloException { + if (!exists(tableName)) + throw new TableNotFoundException(tableName, tableName, ""); + } + + @Override + public void compact(String tableName, Text start, Text end, List iterators, boolean flush, boolean wait) throws AccumuloSecurityException, + TableNotFoundException, AccumuloException { + if (!exists(tableName)) + throw new TableNotFoundException(tableName, tableName, ""); + + if (iterators != null && iterators.size() > 0) + throw new UnsupportedOperationException(); + } + + @Override + public void compact(String tableName, CompactionConfig config) throws AccumuloSecurityException, TableNotFoundException, AccumuloException { + if (!exists(tableName)) + throw new TableNotFoundException(tableName, tableName, ""); + + if (config.getIterators().size() > 0 || config.getCompactionStrategy() != null) + throw new UnsupportedOperationException("Mock does not support iterators or compaction strategies for compactions"); + } + + @Override + public void cancelCompaction(String tableName) throws AccumuloSecurityException, TableNotFoundException, AccumuloException { + if (!exists(tableName)) + throw new TableNotFoundException(tableName, tableName, ""); + } + + @Override + public void clone(String srcTableName, String newTableName, boolean flush, Map propertiesToSet, Set propertiesToExclude) + throws AccumuloException, AccumuloSecurityException, TableNotFoundException, TableExistsException { + throw new NotImplementedException(); + } + + @Override + public void flush(String tableName, Text start, Text end, boolean wait) throws AccumuloException, AccumuloSecurityException, TableNotFoundException { + if (!exists(tableName)) + throw new TableNotFoundException(tableName, tableName, ""); + } + + @Override + public Text getMaxRow(String tableName, Authorizations auths, Text startRow, boolean startInclusive, Text endRow, boolean endInclusive) + throws TableNotFoundException, AccumuloException, AccumuloSecurityException { + MockTable table = acu.tables.get(tableName); + if (table == null) + throw new TableNotFoundException(tableName, tableName, "no such table"); + + return FindMax.findMax(new MockScanner(table, auths), startRow, startInclusive, endRow, endInclusive); + } + + @Override + public void importTable(String tableName, String exportDir) throws TableExistsException, AccumuloException, AccumuloSecurityException { + throw new NotImplementedException(); + } + + @Override + public void exportTable(String tableName, String exportDir) throws TableNotFoundException, AccumuloException, AccumuloSecurityException { + throw new NotImplementedException(); + } + + @Override + public boolean testClassLoad(String tableName, String className, String asTypeName) throws AccumuloException, AccumuloSecurityException, + TableNotFoundException { + + try { + AccumuloVFSClassLoader.loadClass(className, Class.forName(asTypeName)); + } catch (ClassNotFoundException e) { + log.warn("Could not load class '" + className + "' with type name '" + asTypeName + "' in testClassLoad().", e); + return false; + } + return true; + } + + @Override + public void setSamplerConfiguration(String tableName, SamplerConfiguration samplerConfiguration) throws TableNotFoundException, AccumuloException, + AccumuloSecurityException { + throw new UnsupportedOperationException(); + } + + @Override + public void clearSamplerConfiguration(String tableName) throws TableNotFoundException, AccumuloException, AccumuloSecurityException { + throw new UnsupportedOperationException(); + } + + @Override + public SamplerConfiguration getSamplerConfiguration(String tableName) throws TableNotFoundException, AccumuloException, AccumuloSecurityException { + throw new UnsupportedOperationException(); + } + + @Override + public Locations locate(String tableName, Collection ranges) throws AccumuloException, AccumuloSecurityException, TableNotFoundException { + throw new UnsupportedOperationException(); + } +} http://git-wip-us.apache.org/repos/asf/accumulo/blob/4f2e6472/core/src/main/java/org/apache/accumulo/core/client/mock/MockUser.java ---------------------------------------------------------------------- diff --git a/core/src/main/java/org/apache/accumulo/core/client/mock/MockUser.java b/core/src/main/java/org/apache/accumulo/core/client/mock/MockUser.java new file mode 100644 index 0000000..e32edad --- /dev/null +++ b/core/src/main/java/org/apache/accumulo/core/client/mock/MockUser.java @@ -0,0 +1,41 @@ +/* + * 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.accumulo.core.client.mock; + +import java.util.EnumSet; + +import org.apache.accumulo.core.client.security.tokens.AuthenticationToken; +import org.apache.accumulo.core.security.Authorizations; +import org.apache.accumulo.core.security.SystemPermission; + +/** + * @deprecated since 1.8.0; use MiniAccumuloCluster or a standard mock framework instead. + */ +@Deprecated +public class MockUser { + final EnumSet permissions; + final String name; + AuthenticationToken token; + Authorizations authorizations; + + MockUser(String principal, AuthenticationToken token, Authorizations auths) { + this.name = principal; + this.token = token.clone(); + this.authorizations = auths; + this.permissions = EnumSet.noneOf(SystemPermission.class); + } +} http://git-wip-us.apache.org/repos/asf/accumulo/blob/4f2e6472/core/src/main/java/org/apache/accumulo/core/client/mock/impl/MockTabletLocator.java ---------------------------------------------------------------------- diff --git a/core/src/main/java/org/apache/accumulo/core/client/mock/impl/MockTabletLocator.java b/core/src/main/java/org/apache/accumulo/core/client/mock/impl/MockTabletLocator.java new file mode 100644 index 0000000..a52af79 --- /dev/null +++ b/core/src/main/java/org/apache/accumulo/core/client/mock/impl/MockTabletLocator.java @@ -0,0 +1,75 @@ +/* + * 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.accumulo.core.client.mock.impl; + +import java.util.Collection; +import java.util.Collections; +import java.util.List; +import java.util.Map; + +import org.apache.accumulo.core.client.AccumuloException; +import org.apache.accumulo.core.client.AccumuloSecurityException; +import org.apache.accumulo.core.client.Instance; +import org.apache.accumulo.core.client.TableNotFoundException; +import org.apache.accumulo.core.client.impl.ClientContext; +import org.apache.accumulo.core.client.impl.TabletLocator; +import org.apache.accumulo.core.data.Mutation; +import org.apache.accumulo.core.data.Range; +import org.apache.accumulo.core.data.impl.KeyExtent; +import org.apache.hadoop.io.Text; + +/** + * @deprecated since 1.8.0; use MiniAccumuloCluster or a standard mock framework instead. + */ +@Deprecated +public class MockTabletLocator extends TabletLocator { + public MockTabletLocator() {} + + @Override + public TabletLocation locateTablet(ClientContext context, Text row, boolean skipRow, boolean retry) throws AccumuloException, AccumuloSecurityException, + TableNotFoundException { + throw new UnsupportedOperationException(); + } + + @Override + public void binMutations(ClientContext context, List mutations, Map> binnedMutations, List failures) + throws AccumuloException, AccumuloSecurityException, TableNotFoundException { + TabletServerMutations tsm = new TabletServerMutations<>("5"); + for (T m : mutations) + tsm.addMutation(new KeyExtent(), m); + binnedMutations.put("", tsm); + } + + @Override + public List binRanges(ClientContext context, List ranges, Map>> binnedRanges) throws AccumuloException, + AccumuloSecurityException, TableNotFoundException { + binnedRanges.put("", Collections.singletonMap(new KeyExtent("", null, null), ranges)); + return Collections.emptyList(); + } + + @Override + public void invalidateCache(KeyExtent failedExtent) {} + + @Override + public void invalidateCache(Collection keySet) {} + + @Override + public void invalidateCache() {} + + @Override + public void invalidateCache(Instance instance, String server) {} +} http://git-wip-us.apache.org/repos/asf/accumulo/blob/4f2e6472/core/src/main/java/org/apache/accumulo/core/client/mock/package-info.java ---------------------------------------------------------------------- diff --git a/core/src/main/java/org/apache/accumulo/core/client/mock/package-info.java b/core/src/main/java/org/apache/accumulo/core/client/mock/package-info.java new file mode 100644 index 0000000..cdd5593 --- /dev/null +++ b/core/src/main/java/org/apache/accumulo/core/client/mock/package-info.java @@ -0,0 +1,25 @@ +/* + * 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. + */ +/** + * Mock framework for Accumulo + * + *

+ * Deprecated since 1.8.0; use MiniAccumuloCluster or a standard mock framework instead. + */ +@Deprecated +package org.apache.accumulo.core.client.mock; + http://git-wip-us.apache.org/repos/asf/accumulo/blob/4f2e6472/core/src/main/java/org/apache/accumulo/core/client/rfile/RFileScanner.java ---------------------------------------------------------------------- diff --git a/core/src/main/java/org/apache/accumulo/core/client/rfile/RFileScanner.java b/core/src/main/java/org/apache/accumulo/core/client/rfile/RFileScanner.java index 7b747d1..4dfba68 100644 --- a/core/src/main/java/org/apache/accumulo/core/client/rfile/RFileScanner.java +++ b/core/src/main/java/org/apache/accumulo/core/client/rfile/RFileScanner.java @@ -26,6 +26,7 @@ import java.util.List; import java.util.Map.Entry; import java.util.Set; import java.util.SortedSet; +import java.util.concurrent.TimeUnit; import org.apache.accumulo.core.client.IteratorSetting; import org.apache.accumulo.core.client.Scanner; @@ -155,6 +156,24 @@ class RFileScanner extends ScannerOptions implements Scanner { throw new UnsupportedOperationException(); } + @Deprecated + @Override + public void setTimeOut(int timeOut) { + if (timeOut == Integer.MAX_VALUE) + setTimeout(Long.MAX_VALUE, TimeUnit.MILLISECONDS); + else + setTimeout(timeOut, TimeUnit.SECONDS); + } + + @Deprecated + @Override + public int getTimeOut() { + long timeout = getTimeout(TimeUnit.SECONDS); + if (timeout >= Integer.MAX_VALUE) + return Integer.MAX_VALUE; + return (int) timeout; + } + @Override public void setRange(Range range) { this.range = range; http://git-wip-us.apache.org/repos/asf/accumulo/blob/4f2e6472/core/src/main/java/org/apache/accumulo/core/client/security/tokens/KerberosToken.java ---------------------------------------------------------------------- diff --git a/core/src/main/java/org/apache/accumulo/core/client/security/tokens/KerberosToken.java b/core/src/main/java/org/apache/accumulo/core/client/security/tokens/KerberosToken.java index 26f2d02..1a4869d 100644 --- a/core/src/main/java/org/apache/accumulo/core/client/security/tokens/KerberosToken.java +++ b/core/src/main/java/org/apache/accumulo/core/client/security/tokens/KerberosToken.java @@ -69,10 +69,33 @@ public class KerberosToken implements AuthenticationToken { * A keytab file containing the principal's credentials. */ public KerberosToken(String principal, File keytab) throws IOException { + this(principal, keytab, false); + } + + /** + * Creates a token and logs in via {@link UserGroupInformation} using the provided principal and keytab. A key for the principal must exist in the keytab, + * otherwise login will fail. + * + * @param principal + * The Kerberos principal + * @param keytab + * A keytab file + * @param replaceCurrentUser + * Should the current Hadoop user be replaced with this user + * @deprecated since 1.8.0, @see #KerberosToken(String, File) + */ + @Deprecated + public KerberosToken(String principal, File keytab, boolean replaceCurrentUser) throws IOException { requireNonNull(principal, "Principal was null"); requireNonNull(keytab, "Keytab was null"); checkArgument(keytab.exists() && keytab.isFile(), "Keytab was not a normal file"); - UserGroupInformation ugi = UserGroupInformation.loginUserFromKeytabAndReturnUGI(principal, keytab.getAbsolutePath()); + UserGroupInformation ugi; + if (replaceCurrentUser) { + UserGroupInformation.loginUserFromKeytab(principal, keytab.getAbsolutePath()); + ugi = UserGroupInformation.getCurrentUser(); + } else { + ugi = UserGroupInformation.loginUserFromKeytabAndReturnUGI(principal, keytab.getAbsolutePath()); + } this.principal = ugi.getUserName(); this.keytab = keytab; } http://git-wip-us.apache.org/repos/asf/accumulo/blob/4f2e6472/core/src/main/java/org/apache/accumulo/core/conf/Property.java ---------------------------------------------------------------------- diff --git a/core/src/main/java/org/apache/accumulo/core/conf/Property.java b/core/src/main/java/org/apache/accumulo/core/conf/Property.java index 143b823..5f4f715 100644 --- a/core/src/main/java/org/apache/accumulo/core/conf/Property.java +++ b/core/src/main/java/org/apache/accumulo/core/conf/Property.java @@ -159,6 +159,9 @@ public enum Property { */ INSTANCE_RPC_SASL_ENABLED("instance.rpc.sasl.enabled", "false", PropertyType.BOOLEAN, "Configures Thrift RPCs to require SASL with GSSAPI which supports Kerberos authentication. Mutually exclusive with SSL RPC configuration."), + @Deprecated + INSTANCE_RPC_SASL_PROXYUSERS("instance.rpc.sasl.impersonation.", null, PropertyType.PREFIX, + "Prefix that allows configuration of users that are allowed to impersonate other users"), INSTANCE_RPC_SASL_ALLOWED_USER_IMPERSONATION("instance.rpc.sasl.allowed.user.impersonation", "", PropertyType.STRING, "One-line configuration property controlling what users are allowed to impersonate other users"), INSTANCE_RPC_SASL_ALLOWED_HOST_IMPERSONATION("instance.rpc.sasl.allowed.host.impersonation", "", PropertyType.STRING, @@ -242,6 +245,12 @@ public enum Property { TSERV_INDEXCACHE_SIZE("tserver.cache.index.size", "512M", PropertyType.MEMORY, "Specifies the size of the cache for file indices."), TSERV_PORTSEARCH("tserver.port.search", "false", PropertyType.BOOLEAN, "if the ports above are in use, search higher ports until one is available"), TSERV_CLIENTPORT("tserver.port.client", "9997", PropertyType.PORT, "The port used for handling client connections on the tablet servers"), + @Deprecated + TSERV_MUTATION_QUEUE_MAX("tserver.mutation.queue.max", "1M", PropertyType.MEMORY, "This setting is deprecated. See tserver.total.mutation.queue.max. " + + "The amount of memory to use to store write-ahead-log mutations-per-session before flushing them. Since the buffer is per write session, consider the" + + " max number of concurrent writer when configuring. When using Hadoop 2, Accumulo will call hsync() on the WAL . For a small number of " + + "concurrent writers, increasing this buffer size decreases the frequncy of hsync calls. For a large number of concurrent writers a small buffers " + + "size is ok because of group commit."), TSERV_TOTAL_MUTATION_QUEUE_MAX("tserver.total.mutation.queue.max", "50M", PropertyType.MEMORY, "The amount of memory used to store write-ahead-log mutations before flushing them."), TSERV_TABLET_SPLIT_FINDMIDPOINT_MAXOPEN("tserver.tablet.split.midpoint.files.max", "300", PropertyType.COUNT, @@ -335,6 +344,8 @@ public enum Property { "The number of threads for the distributed work queue. These threads are used for copying failed bulk files."), TSERV_WAL_SYNC("tserver.wal.sync", "true", PropertyType.BOOLEAN, "Use the SYNC_BLOCK create flag to sync WAL writes to disk. Prevents problems recovering from sudden system resets."), + @Deprecated + TSERV_WAL_SYNC_METHOD("tserver.wal.sync.method", "hsync", PropertyType.STRING, "This property is deprecated. Use table.durability instead."), TSERV_ASSIGNMENT_DURATION_WARNING("tserver.assignment.duration.warning", "10m", PropertyType.TIMEDURATION, "The amount of time an assignment can run " + " before the server will print a warning along with the current stack trace. Meant to help debug stuck assignments"), TSERV_REPLICATION_REPLAYERS("tserver.replication.replayer.", null, PropertyType.PREFIX, @@ -454,6 +465,8 @@ public enum Property { "Determines the max # of files each tablet in a table can have. When adjusting this property you may want to consider adjusting" + " table.compaction.major.ratio also. Setting this property to 0 will make it default to tserver.scan.files.open.max-1, this will prevent a" + " tablet from having more files than can be opened. Setting this property low may throttle ingest and increase query performance."), + @Deprecated + TABLE_WALOG_ENABLED("table.walog.enabled", "true", PropertyType.BOOLEAN, "This setting is deprecated. Use table.durability=none instead."), TABLE_BLOOM_ENABLED("table.bloom.enabled", "false", PropertyType.BOOLEAN, "Use bloom filters on this table."), TABLE_BLOOM_LOAD_THRESHOLD("table.bloom.load.threshold", "1", PropertyType.COUNT, "This number of seeks that would actually use a bloom filter must occur before a file's bloom filter is loaded." http://git-wip-us.apache.org/repos/asf/accumulo/blob/4f2e6472/core/src/main/java/org/apache/accumulo/core/constraints/Constraint.java ---------------------------------------------------------------------- diff --git a/core/src/main/java/org/apache/accumulo/core/constraints/Constraint.java b/core/src/main/java/org/apache/accumulo/core/constraints/Constraint.java index b70afc6..a936ef5 100644 --- a/core/src/main/java/org/apache/accumulo/core/constraints/Constraint.java +++ b/core/src/main/java/org/apache/accumulo/core/constraints/Constraint.java @@ -21,6 +21,7 @@ import java.util.List; import org.apache.accumulo.core.data.Mutation; import org.apache.accumulo.core.data.impl.KeyExtent; import org.apache.accumulo.core.security.AuthorizationContainer; +import org.apache.accumulo.core.security.Authorizations; /** * Constraint objects are used to determine if mutations will be applied to a table. @@ -61,6 +62,15 @@ public interface Constraint { * Gets the authorizations in the environment. * * @return authorizations + * @deprecated Use {@link #getAuthorizationsContainer()} instead. + */ + @Deprecated + Authorizations getAuthorizations(); + + /** + * Gets the authorizations in the environment. + * + * @return authorizations */ AuthorizationContainer getAuthorizationsContainer(); } http://git-wip-us.apache.org/repos/asf/accumulo/blob/4f2e6472/core/src/main/java/org/apache/accumulo/core/data/ComparableBytes.java ---------------------------------------------------------------------- diff --git a/core/src/main/java/org/apache/accumulo/core/data/ComparableBytes.java b/core/src/main/java/org/apache/accumulo/core/data/ComparableBytes.java new file mode 100644 index 0000000..78c0e56 --- /dev/null +++ b/core/src/main/java/org/apache/accumulo/core/data/ComparableBytes.java @@ -0,0 +1,58 @@ +/* + * 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.accumulo.core.data; + +import org.apache.hadoop.io.BinaryComparable; + +/** + * An array of bytes wrapped so as to extend Hadoop's BinaryComparable class. + * + * @deprecated since 1.7.0 In an attempt to clean up types in the data package that were not intended to be in public API this type was deprecated. Technically + * this method was not considered part of the public API in 1.6.0 and earlier, therefore it could have been deleted. However a decision was made to + * deprecate in order to be cautious and avoid confusion between 1.6.0 and 1.7.0. + */ +@Deprecated +public class ComparableBytes extends BinaryComparable { + + public byte[] data; + + /** + * Creates a new byte wrapper. The given byte array is used directly as a backing array, so later changes made to the array reflect into the new object. + * + * @param b + * bytes to wrap + */ + public ComparableBytes(byte[] b) { + this.data = b; + } + + /** + * Gets the wrapped bytes in this object. + * + * @return bytes + */ + @Override + public byte[] getBytes() { + return data; + } + + @Override + public int getLength() { + return data.length; + } + +} http://git-wip-us.apache.org/repos/asf/accumulo/blob/4f2e6472/core/src/main/java/org/apache/accumulo/core/data/KeyExtent.java ---------------------------------------------------------------------- diff --git a/core/src/main/java/org/apache/accumulo/core/data/KeyExtent.java b/core/src/main/java/org/apache/accumulo/core/data/KeyExtent.java new file mode 100644 index 0000000..4e3d058 --- /dev/null +++ b/core/src/main/java/org/apache/accumulo/core/data/KeyExtent.java @@ -0,0 +1,259 @@ +/* + * 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.accumulo.core.data; + +import java.io.DataInput; +import java.io.DataOutput; +import java.io.IOException; +import java.util.Collection; +import java.util.Map.Entry; +import java.util.Set; +import java.util.SortedMap; +import java.util.SortedSet; +import java.util.TreeMap; +import java.util.TreeSet; +import java.util.UUID; + +import org.apache.accumulo.core.data.thrift.TKeyExtent; +import org.apache.accumulo.core.metadata.schema.MetadataSchema; +import org.apache.hadoop.io.BinaryComparable; +import org.apache.hadoop.io.Text; +import org.apache.hadoop.io.WritableComparable; + +/** + * keeps track of information needed to identify a tablet + * + * @deprecated since 1.7.0 use {@link TabletId} + */ +@Deprecated +public class KeyExtent implements WritableComparable { + + // Wrapping impl.KeyExtent to resuse code. Did not want to extend impl.KeyExtent because any changes to impl.KeyExtent would be reflected in this class. + // Wrapping impl.KeyExtent allows the API of this deprecated class to be frozen. + private org.apache.accumulo.core.data.impl.KeyExtent wrapped; + + public KeyExtent() { + this.wrapped = new org.apache.accumulo.core.data.impl.KeyExtent(); + } + + public KeyExtent(Text table, Text endRow, Text prevEndRow) { + this.wrapped = new org.apache.accumulo.core.data.impl.KeyExtent(table.toString(), endRow, prevEndRow); + } + + public KeyExtent(KeyExtent extent) { + this.wrapped = new org.apache.accumulo.core.data.impl.KeyExtent(extent.getTableId().toString(), extent.getEndRow(), extent.getPrevEndRow()); + } + + public KeyExtent(TKeyExtent tke) { + this.wrapped = new org.apache.accumulo.core.data.impl.KeyExtent(tke); + } + + // constructor for loading extents from metadata rows + public KeyExtent(Text flattenedExtent, Value prevEndRow) { + this.wrapped = new org.apache.accumulo.core.data.impl.KeyExtent(flattenedExtent, prevEndRow); + } + + // recreates an encoded extent from a string representation + // this encoding is what is stored as the row id of the metadata table + public KeyExtent(Text flattenedExtent, Text prevEndRow) { + this.wrapped = new org.apache.accumulo.core.data.impl.KeyExtent(flattenedExtent, prevEndRow); + } + + public Text getMetadataEntry() { + return wrapped.getMetadataEntry(); + } + + public void setTableId(Text tId) { + wrapped.setTableId(tId.toString()); + } + + public Text getTableId() { + return new Text(wrapped.getTableId()); + } + + public void setEndRow(Text endRow) { + wrapped.setEndRow(endRow); + } + + public Text getEndRow() { + return wrapped.getEndRow(); + } + + public Text getPrevEndRow() { + return wrapped.getPrevEndRow(); + } + + public void setPrevEndRow(Text prevEndRow) { + wrapped.setPrevEndRow(prevEndRow); + } + + @Override + public void readFields(DataInput in) throws IOException { + wrapped.readFields(in); + } + + @Override + public void write(DataOutput out) throws IOException { + wrapped.write(out); + } + + public Mutation getPrevRowUpdateMutation() { + return wrapped.getPrevRowUpdateMutation(); + } + + @Override + public int compareTo(KeyExtent other) { + return wrapped.compareTo(other.wrapped); + } + + @Override + public int hashCode() { + return wrapped.hashCode(); + } + + @Override + public boolean equals(Object o) { + if (o instanceof KeyExtent) { + return wrapped.equals(((KeyExtent) o).wrapped); + } + + return false; + } + + @Override + public String toString() { + return wrapped.toString(); + } + + public UUID getUUID() { + return wrapped.getUUID(); + } + + public boolean contains(ByteSequence bsrow) { + return wrapped.contains(bsrow); + } + + public boolean contains(BinaryComparable row) { + return wrapped.contains(row); + } + + public Range toDataRange() { + return wrapped.toDataRange(); + } + + public Range toMetadataRange() { + return wrapped.toMetadataRange(); + } + + public boolean overlaps(KeyExtent other) { + return wrapped.overlaps(other.wrapped); + } + + public TKeyExtent toThrift() { + return wrapped.toThrift(); + } + + public boolean isPreviousExtent(KeyExtent prevExtent) { + return wrapped.isPreviousExtent(prevExtent.wrapped); + } + + public boolean isMeta() { + return wrapped.isMeta(); + } + + public boolean isRootTablet() { + return wrapped.isRootTablet(); + } + + private static SortedSet unwrap(Set tablets) { + SortedSet trans = new TreeSet<>(); + for (KeyExtent wrapper : tablets) { + trans.add(wrapper.wrapped); + } + + return trans; + } + + private static KeyExtent wrap(org.apache.accumulo.core.data.impl.KeyExtent ke) { + return new KeyExtent(new Text(ke.getTableId()), ke.getEndRow(), ke.getPrevEndRow()); + } + + private static SortedSet wrap(Collection unwrapped) { + SortedSet wrapped = new TreeSet<>(); + for (org.apache.accumulo.core.data.impl.KeyExtent wrappee : unwrapped) { + wrapped.add(wrap(wrappee)); + } + + return wrapped; + } + + public static Text getMetadataEntry(Text tableId, Text endRow) { + return MetadataSchema.TabletsSection.getRow(tableId.toString(), endRow); + } + + /** + * Empty start or end rows tell the method there are no start or end rows, and to use all the keyextents that are before the end row if no start row etc. + * + * @deprecated this method not intended for public use and is likely to be removed in a future version. + * @return all the key extents that the rows cover + */ + @Deprecated + public static Collection getKeyExtentsForRange(Text startRow, Text endRow, Set kes) { + return wrap(org.apache.accumulo.core.data.impl.KeyExtent.getKeyExtentsForRange(startRow, endRow, unwrap(kes))); + } + + public static Text decodePrevEndRow(Value ibw) { + return org.apache.accumulo.core.data.impl.KeyExtent.decodePrevEndRow(ibw); + } + + public static Value encodePrevEndRow(Text per) { + return org.apache.accumulo.core.data.impl.KeyExtent.encodePrevEndRow(per); + } + + public static Mutation getPrevRowUpdateMutation(KeyExtent ke) { + return org.apache.accumulo.core.data.impl.KeyExtent.getPrevRowUpdateMutation(ke.wrapped); + } + + public static byte[] tableOfMetadataRow(Text row) { + return org.apache.accumulo.core.data.impl.KeyExtent.tableOfMetadataRow(row); + } + + public static SortedSet findChildren(KeyExtent ke, SortedSet tablets) { + return wrap(org.apache.accumulo.core.data.impl.KeyExtent.findChildren(ke.wrapped, unwrap(tablets))); + } + + public static KeyExtent findContainingExtent(KeyExtent extent, SortedSet extents) { + return wrap(org.apache.accumulo.core.data.impl.KeyExtent.findContainingExtent(extent.wrapped, unwrap(extents))); + } + + public static Set findOverlapping(KeyExtent nke, SortedSet extents) { + return wrap(org.apache.accumulo.core.data.impl.KeyExtent.findOverlapping(nke.wrapped, unwrap(extents))); + } + + public static Set findOverlapping(KeyExtent nke, SortedMap extents) { + SortedMap trans = new TreeMap<>(); + for (Entry entry : extents.entrySet()) { + trans.put(entry.getKey().wrapped, entry.getValue()); + } + + return wrap(org.apache.accumulo.core.data.impl.KeyExtent.findOverlapping(nke.wrapped, trans)); + } + + public static Text getMetadataEntry(KeyExtent extent) { + return org.apache.accumulo.core.data.impl.KeyExtent.getMetadataEntry(extent.wrapped); + } +} http://git-wip-us.apache.org/repos/asf/accumulo/blob/4f2e6472/core/src/main/java/org/apache/accumulo/core/data/PartialKey.java ---------------------------------------------------------------------- diff --git a/core/src/main/java/org/apache/accumulo/core/data/PartialKey.java b/core/src/main/java/org/apache/accumulo/core/data/PartialKey.java index 8ff0017..bf0df1e 100644 --- a/core/src/main/java/org/apache/accumulo/core/data/PartialKey.java +++ b/core/src/main/java/org/apache/accumulo/core/data/PartialKey.java @@ -31,6 +31,24 @@ public enum PartialKey { } /** + * Get a partial key specification by depth of the specification. + * + * @param depth + * depth of scope (i.e., number of fields included) + * @return partial key + * @throws IllegalArgumentException + * if no partial key has the given depth + * @deprecated since 1.7.0 + */ + @Deprecated + public static PartialKey getByDepth(int depth) { + for (PartialKey d : PartialKey.values()) + if (depth == d.depth) + return d; + throw new IllegalArgumentException("Invalid legacy depth " + depth); + } + + /** * Gets the depth of this partial key. * * @return depth http://git-wip-us.apache.org/repos/asf/accumulo/blob/4f2e6472/core/src/main/java/org/apache/accumulo/core/data/Value.java ---------------------------------------------------------------------- diff --git a/core/src/main/java/org/apache/accumulo/core/data/Value.java b/core/src/main/java/org/apache/accumulo/core/data/Value.java index 9c63a6a..95c3c70 100644 --- a/core/src/main/java/org/apache/accumulo/core/data/Value.java +++ b/core/src/main/java/org/apache/accumulo/core/data/Value.java @@ -25,6 +25,8 @@ import java.io.DataInput; import java.io.DataOutput; import java.io.IOException; import java.nio.ByteBuffer; +import java.nio.charset.StandardCharsets; +import java.util.List; import org.apache.hadoop.io.BytesWritable; import org.apache.hadoop.io.Text; @@ -56,7 +58,7 @@ public class Value implements WritableComparable { * @since 1.8.0 */ public Value(CharSequence cs) { - this(cs.toString().getBytes(UTF_8)); + this(cs.toString().getBytes(StandardCharsets.UTF_8)); } /** @@ -94,6 +96,20 @@ public class Value implements WritableComparable { } /** + * @deprecated A copy of the bytes in the buffer is always made. Use {@link #Value(ByteBuffer)} instead. + * + * @param bytes + * bytes of value (may not be null) + * @param copy + * false to use the backing array of the buffer directly as the backing array, true to force a copy + */ + @Deprecated + public Value(ByteBuffer bytes, boolean copy) { + /* TODO ACCUMULO-2509 right now this uses the entire backing array, which must be accessible. */ + this(toBytes(bytes), false); + } + + /** * Creates a Value using a byte array as the initial value. * * @param bytes @@ -262,4 +278,22 @@ public class Value implements WritableComparable { WritableComparator.define(Value.class, new Comparator()); } + /** + * Converts a list of byte arrays to a two-dimensional array. + * + * @param array + * list of byte arrays + * @return two-dimensional byte array containing one given byte array per row + * @deprecated since 1.7.0; this utility method is not appropriate for the {@link Value} object + */ + @Deprecated + public static byte[][] toArray(final List array) { + // List#toArray doesn't work on lists of byte []. + byte[][] results = new byte[array.size()][]; + for (int i = 0; i < array.size(); i++) { + results[i] = array.get(i); + } + return results; + } + } http://git-wip-us.apache.org/repos/asf/accumulo/blob/4f2e6472/core/src/main/java/org/apache/accumulo/core/data/impl/KeyExtent.java ---------------------------------------------------------------------- diff --git a/core/src/main/java/org/apache/accumulo/core/data/impl/KeyExtent.java b/core/src/main/java/org/apache/accumulo/core/data/impl/KeyExtent.java index 304abb8..dcb8eb7 100644 --- a/core/src/main/java/org/apache/accumulo/core/data/impl/KeyExtent.java +++ b/core/src/main/java/org/apache/accumulo/core/data/impl/KeyExtent.java @@ -25,6 +25,8 @@ import java.io.DataOutputStream; import java.io.IOException; import java.lang.ref.WeakReference; import java.nio.ByteBuffer; +import java.util.ArrayList; +import java.util.Collection; import java.util.Collections; import java.util.Map.Entry; import java.util.Set; @@ -293,6 +295,77 @@ public class KeyExtent implements WritableComparable { return getPrevRowUpdateMutation(this); } + /** + * Empty start or end rows tell the method there are no start or end rows, and to use all the keyextents that are before the end row if no start row etc. + * + * @deprecated this method not intended for public use and is likely to be removed in a future version. + * @return all the key extents that the rows cover + */ + @Deprecated + public static Collection getKeyExtentsForRange(Text startRow, Text endRow, Set kes) { + if (kes == null) + return Collections.emptyList(); + if (startRow == null) + startRow = new Text(); + if (endRow == null) + endRow = new Text(); + Collection keys = new ArrayList<>(); + for (KeyExtent ckes : kes) { + if (ckes.getPrevEndRow() == null) { + if (ckes.getEndRow() == null) { + // only tablet + keys.add(ckes); + } else { + // first tablet + // if start row = '' then we want everything up to the endRow which will always include the first tablet + if (startRow.getLength() == 0) { + keys.add(ckes); + } else if (ckes.getEndRow().compareTo(startRow) >= 0) { + keys.add(ckes); + } + } + } else { + if (ckes.getEndRow() == null) { + // last tablet + // if endRow = '' and we're at the last tablet, add it + if (endRow.getLength() == 0) { + keys.add(ckes); + } + if (ckes.getPrevEndRow().compareTo(endRow) < 0) { + keys.add(ckes); + } + } else { + // tablet in the middle + if (startRow.getLength() == 0) { + // no start row + + if (endRow.getLength() == 0) { + // no start & end row + keys.add(ckes); + } else { + // just no start row + if (ckes.getPrevEndRow().compareTo(endRow) < 0) { + keys.add(ckes); + } + } + } else if (endRow.getLength() == 0) { + // no end row + if (ckes.getEndRow().compareTo(startRow) >= 0) { + keys.add(ckes); + } + } else { + // no null prevend or endrows and no empty string start or end rows + if (ckes.getPrevEndRow().compareTo(endRow) < 0 && ckes.getEndRow().compareTo(startRow) >= 0) { + keys.add(ckes); + } + } + + } + } + } + return keys; + } + public static Text decodePrevEndRow(Value ibw) { Text per = null; http://git-wip-us.apache.org/repos/asf/accumulo/blob/4f2e6472/core/src/main/java/org/apache/accumulo/core/data/impl/TabletIdImpl.java ---------------------------------------------------------------------- diff --git a/core/src/main/java/org/apache/accumulo/core/data/impl/TabletIdImpl.java b/core/src/main/java/org/apache/accumulo/core/data/impl/TabletIdImpl.java index d34e379..d28a1ee 100644 --- a/core/src/main/java/org/apache/accumulo/core/data/impl/TabletIdImpl.java +++ b/core/src/main/java/org/apache/accumulo/core/data/impl/TabletIdImpl.java @@ -17,6 +17,8 @@ package org.apache.accumulo.core.data.impl; +import java.util.function.Function; + import org.apache.accumulo.core.data.Range; import org.apache.accumulo.core.data.TabletId; import org.apache.hadoop.io.Text; @@ -25,6 +27,35 @@ public class TabletIdImpl implements TabletId { private KeyExtent ke; + @SuppressWarnings("deprecation") + public static final Function KE_2_TID_OLD = new Function() { + @Override + public TabletId apply(org.apache.accumulo.core.data.KeyExtent input) { + // the following if null check is to appease findbugs... grumble grumble spent a good part of my morning looking into this + // http://sourceforge.net/p/findbugs/bugs/1139/ + // https://code.google.com/p/guava-libraries/issues/detail?id=920 + if (input == null) + return null; + return new TabletIdImpl(input); + } + }; + + @SuppressWarnings("deprecation") + public static final Function TID_2_KE_OLD = new Function() { + @Override + public org.apache.accumulo.core.data.KeyExtent apply(TabletId input) { + if (input == null) + return null; + return new org.apache.accumulo.core.data.KeyExtent(input.getTableId(), input.getEndRow(), input.getPrevEndRow()); + } + + }; + + @Deprecated + public TabletIdImpl(org.apache.accumulo.core.data.KeyExtent ke) { + this.ke = new KeyExtent(ke.getTableId().toString(), ke.getEndRow(), ke.getPrevEndRow()); + } + public TabletIdImpl(KeyExtent ke) { this.ke = ke; } http://git-wip-us.apache.org/repos/asf/accumulo/blob/4f2e6472/core/src/main/java/org/apache/accumulo/core/iterators/AggregatingIterator.java ---------------------------------------------------------------------- diff --git a/core/src/main/java/org/apache/accumulo/core/iterators/AggregatingIterator.java b/core/src/main/java/org/apache/accumulo/core/iterators/AggregatingIterator.java new file mode 100644 index 0000000..979eaeb --- /dev/null +++ b/core/src/main/java/org/apache/accumulo/core/iterators/AggregatingIterator.java @@ -0,0 +1,215 @@ +/* + * 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.accumulo.core.iterators; + +import java.io.IOException; +import java.util.Collection; +import java.util.Collections; +import java.util.Map; +import java.util.Map.Entry; + +import org.apache.accumulo.core.conf.Property; +import org.apache.accumulo.core.data.ByteSequence; +import org.apache.accumulo.core.data.Key; +import org.apache.accumulo.core.data.PartialKey; +import org.apache.accumulo.core.data.Range; +import org.apache.accumulo.core.data.Value; +import org.apache.accumulo.core.iterators.conf.ColumnToClassMapping; +import org.apache.accumulo.start.classloader.vfs.AccumuloVFSClassLoader; +import org.slf4j.Logger; +import org.slf4j.LoggerFactory; + +/** + * This iterator wraps another iterator. It automatically aggregates. + * + * @deprecated since 1.4, replaced by {@link org.apache.accumulo.core.iterators.Combiner} + */ + +@Deprecated +public class AggregatingIterator implements SortedKeyValueIterator, OptionDescriber { + + private SortedKeyValueIterator iterator; + private ColumnToClassMapping aggregators; + + private Key workKey = new Key(); + + private Key aggrKey; + private Value aggrValue; + // private boolean propogateDeletes; + private static final Logger log = LoggerFactory.getLogger(AggregatingIterator.class); + + @Override + public AggregatingIterator deepCopy(IteratorEnvironment env) { + return new AggregatingIterator(this, env); + } + + private AggregatingIterator(AggregatingIterator other, IteratorEnvironment env) { + iterator = other.iterator.deepCopy(env); + aggregators = other.aggregators; + } + + public AggregatingIterator() {} + + private void aggregateRowColumn(org.apache.accumulo.core.iterators.aggregation.Aggregator aggr) throws IOException { + // this function assumes that first value is not delete + + if (iterator.getTopKey().isDeleted()) + return; + + workKey.set(iterator.getTopKey()); + + Key keyToAggregate = workKey; + + aggr.reset(); + + aggr.collect(iterator.getTopValue()); + iterator.next(); + + while (iterator.hasTop() && !iterator.getTopKey().isDeleted() && iterator.getTopKey().equals(keyToAggregate, PartialKey.ROW_COLFAM_COLQUAL_COLVIS)) { + aggr.collect(iterator.getTopValue()); + iterator.next(); + } + + aggrKey = workKey; + aggrValue = aggr.aggregate(); + + } + + private void findTop() throws IOException { + // check if aggregation is needed + if (iterator.hasTop()) { + org.apache.accumulo.core.iterators.aggregation.Aggregator aggr = aggregators.getObject(iterator.getTopKey()); + if (aggr != null) { + aggregateRowColumn(aggr); + } + } + } + + public AggregatingIterator(SortedKeyValueIterator iterator, + ColumnToClassMapping aggregators) throws IOException { + this.iterator = iterator; + this.aggregators = aggregators; + } + + @Override + public Key getTopKey() { + if (aggrKey != null) { + return aggrKey; + } + return iterator.getTopKey(); + } + + @Override + public Value getTopValue() { + if (aggrKey != null) { + return aggrValue; + } + return iterator.getTopValue(); + } + + @Override + public boolean hasTop() { + return aggrKey != null || iterator.hasTop(); + } + + @Override + public void next() throws IOException { + if (aggrKey != null) { + aggrKey = null; + aggrValue = null; + } else { + iterator.next(); + } + + findTop(); + } + + @Override + public void seek(Range range, Collection columnFamilies, boolean inclusive) throws IOException { + // do not want to seek to the middle of a value that should be + // aggregated... + + Range seekRange = IteratorUtil.maximizeStartKeyTimeStamp(range); + + iterator.seek(seekRange, columnFamilies, inclusive); + findTop(); + + if (range.getStartKey() != null) { + while (hasTop() && getTopKey().equals(range.getStartKey(), PartialKey.ROW_COLFAM_COLQUAL_COLVIS) + && getTopKey().getTimestamp() > range.getStartKey().getTimestamp()) { + // the value has a more recent time stamp, so + // pass it up + // log.debug("skipping "+getTopKey()); + next(); + } + + while (hasTop() && range.beforeStartKey(getTopKey())) { + next(); + } + } + + } + + @Override + public void init(SortedKeyValueIterator source, Map options, IteratorEnvironment env) throws IOException { + + this.iterator = source; + + try { + String context = null; + if (null != env) + context = env.getConfig().get(Property.TABLE_CLASSPATH); + this.aggregators = new ColumnToClassMapping<>(options, org.apache.accumulo.core.iterators.aggregation.Aggregator.class, context); + } catch (ClassNotFoundException e) { + log.error(e.toString()); + throw new IllegalArgumentException(e); + } catch (InstantiationException e) { + log.error(e.toString()); + throw new IllegalArgumentException(e); + } catch (IllegalAccessException e) { + log.error(e.toString()); + throw new IllegalArgumentException(e); + } + } + + @Override + public IteratorOptions describeOptions() { + return new IteratorOptions("agg", "Aggregators apply aggregating functions to values with identical keys", null, + Collections.singletonList(" ")); + } + + @Override + public boolean validateOptions(Map options) { + for (Entry entry : options.entrySet()) { + String classname = entry.getValue(); + if (classname == null) + throw new IllegalArgumentException("classname null"); + Class clazz; + try { + clazz = AccumuloVFSClassLoader.loadClass(classname, org.apache.accumulo.core.iterators.aggregation.Aggregator.class); + clazz.newInstance(); + } catch (ClassNotFoundException e) { + throw new IllegalArgumentException("class not found: " + classname); + } catch (InstantiationException e) { + throw new IllegalArgumentException("instantiation exception: " + classname); + } catch (IllegalAccessException e) { + throw new IllegalArgumentException("illegal access exception: " + classname); + } + } + return true; + } +} http://git-wip-us.apache.org/repos/asf/accumulo/blob/4f2e6472/core/src/main/java/org/apache/accumulo/core/iterators/FamilyIntersectingIterator.java ---------------------------------------------------------------------- diff --git a/core/src/main/java/org/apache/accumulo/core/iterators/FamilyIntersectingIterator.java b/core/src/main/java/org/apache/accumulo/core/iterators/FamilyIntersectingIterator.java new file mode 100644 index 0000000..04102b8 --- /dev/null +++ b/core/src/main/java/org/apache/accumulo/core/iterators/FamilyIntersectingIterator.java @@ -0,0 +1,29 @@ +/* + * 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.accumulo.core.iterators; + +import org.apache.accumulo.core.iterators.user.IndexedDocIterator; + +/** + * This class remains here for backwards compatibility. + * + * @deprecated since 1.4, replaced by {@link org.apache.accumulo.core.iterators.user.IndexedDocIterator} + */ +@Deprecated +public class FamilyIntersectingIterator extends IndexedDocIterator { + +} http://git-wip-us.apache.org/repos/asf/accumulo/blob/4f2e6472/core/src/main/java/org/apache/accumulo/core/iterators/GrepIterator.java ---------------------------------------------------------------------- diff --git a/core/src/main/java/org/apache/accumulo/core/iterators/GrepIterator.java b/core/src/main/java/org/apache/accumulo/core/iterators/GrepIterator.java new file mode 100644 index 0000000..5c44c31 --- /dev/null +++ b/core/src/main/java/org/apache/accumulo/core/iterators/GrepIterator.java @@ -0,0 +1,27 @@ +/* + * 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.accumulo.core.iterators; + +/** + * This class remains here for backwards compatibility. + * + * @deprecated since 1.4, replaced by {@link org.apache.accumulo.core.iterators.user.GrepIterator} + */ +@Deprecated +public class GrepIterator extends org.apache.accumulo.core.iterators.user.GrepIterator { + +} http://git-wip-us.apache.org/repos/asf/accumulo/blob/4f2e6472/core/src/main/java/org/apache/accumulo/core/iterators/IntersectingIterator.java ---------------------------------------------------------------------- diff --git a/core/src/main/java/org/apache/accumulo/core/iterators/IntersectingIterator.java b/core/src/main/java/org/apache/accumulo/core/iterators/IntersectingIterator.java new file mode 100644 index 0000000..5765982 --- /dev/null +++ b/core/src/main/java/org/apache/accumulo/core/iterators/IntersectingIterator.java @@ -0,0 +1,27 @@ +/* + * 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.accumulo.core.iterators; + +/** + * This class remains here for backwards compatibility. + * + * @deprecated since 1.4, replaced by {@link org.apache.accumulo.core.iterators.user.IntersectingIterator} + */ +@Deprecated +public class IntersectingIterator extends org.apache.accumulo.core.iterators.user.IntersectingIterator { + +} http://git-wip-us.apache.org/repos/asf/accumulo/blob/4f2e6472/core/src/main/java/org/apache/accumulo/core/iterators/IteratorUtil.java ---------------------------------------------------------------------- diff --git a/core/src/main/java/org/apache/accumulo/core/iterators/IteratorUtil.java b/core/src/main/java/org/apache/accumulo/core/iterators/IteratorUtil.java index 1d5728b..3f23fd2 100644 --- a/core/src/main/java/org/apache/accumulo/core/iterators/IteratorUtil.java +++ b/core/src/main/java/org/apache/accumulo/core/iterators/IteratorUtil.java @@ -70,6 +70,17 @@ public class IteratorUtil { */ public static enum IteratorScope { majc, minc, scan; + + /** + * Fetch the correct configuration key prefix for the given scope. Throws an IllegalArgumentException if no property exists for the given scope. + * + * @deprecated since 1.7.0 This method returns a type that is not part of the public API and is not guaranteed to be stable. The method was deprecated to + * discourage its use. + */ + @Deprecated + public static Property getProperty(IteratorScope scope) { + return IteratorUtil.getProperty(scope); + } } public static class IterInfoComparator implements Comparator, Serializable { http://git-wip-us.apache.org/repos/asf/accumulo/blob/4f2e6472/core/src/main/java/org/apache/accumulo/core/iterators/LargeRowFilter.java ---------------------------------------------------------------------- diff --git a/core/src/main/java/org/apache/accumulo/core/iterators/LargeRowFilter.java b/core/src/main/java/org/apache/accumulo/core/iterators/LargeRowFilter.java new file mode 100644 index 0000000..75155f9 --- /dev/null +++ b/core/src/main/java/org/apache/accumulo/core/iterators/LargeRowFilter.java @@ -0,0 +1,27 @@ +/* + * 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.accumulo.core.iterators; + +/** + * This class remains here for backwards compatibility. + * + * @deprecated since 1.4, replaced by {@link org.apache.accumulo.core.iterators.user.LargeRowFilter} + */ +@Deprecated +public class LargeRowFilter extends org.apache.accumulo.core.iterators.user.LargeRowFilter { + +} http://git-wip-us.apache.org/repos/asf/accumulo/blob/4f2e6472/core/src/main/java/org/apache/accumulo/core/iterators/RowDeletingIterator.java ---------------------------------------------------------------------- diff --git a/core/src/main/java/org/apache/accumulo/core/iterators/RowDeletingIterator.java b/core/src/main/java/org/apache/accumulo/core/iterators/RowDeletingIterator.java new file mode 100644 index 0000000..ee6989f --- /dev/null +++ b/core/src/main/java/org/apache/accumulo/core/iterators/RowDeletingIterator.java @@ -0,0 +1,27 @@ +/* + * 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.accumulo.core.iterators; + +/** + * This class remains here for backwards compatibility. + * + * @deprecated since 1.4, replaced by {@link org.apache.accumulo.core.iterators.user.RowDeletingIterator} + */ +@Deprecated +public class RowDeletingIterator extends org.apache.accumulo.core.iterators.user.RowDeletingIterator { + +} http://git-wip-us.apache.org/repos/asf/accumulo/blob/4f2e6472/core/src/main/java/org/apache/accumulo/core/iterators/VersioningIterator.java ---------------------------------------------------------------------- diff --git a/core/src/main/java/org/apache/accumulo/core/iterators/VersioningIterator.java b/core/src/main/java/org/apache/accumulo/core/iterators/VersioningIterator.java new file mode 100644 index 0000000..d849275 --- /dev/null +++ b/core/src/main/java/org/apache/accumulo/core/iterators/VersioningIterator.java @@ -0,0 +1,36 @@ +/* + * 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.accumulo.core.iterators; + +import org.apache.accumulo.core.data.Key; +import org.apache.accumulo.core.data.Value; + +/** + * This class remains here for backwards compatibility. + * + * @deprecated since 1.4, replaced by {@link org.apache.accumulo.core.iterators.user.VersioningIterator} + */ +@Deprecated +public class VersioningIterator extends org.apache.accumulo.core.iterators.user.VersioningIterator { + public VersioningIterator() {} + + public VersioningIterator(SortedKeyValueIterator iterator, int maxVersions) { + super(); + this.setSource(iterator); + this.maxVersions = maxVersions; + } +} http://git-wip-us.apache.org/repos/asf/accumulo/blob/4f2e6472/core/src/main/java/org/apache/accumulo/core/iterators/WholeRowIterator.java ---------------------------------------------------------------------- diff --git a/core/src/main/java/org/apache/accumulo/core/iterators/WholeRowIterator.java b/core/src/main/java/org/apache/accumulo/core/iterators/WholeRowIterator.java new file mode 100644 index 0000000..7432a88 --- /dev/null +++ b/core/src/main/java/org/apache/accumulo/core/iterators/WholeRowIterator.java @@ -0,0 +1,27 @@ +/* + * 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.accumulo.core.iterators; + +/** + * This class remains here for backwards compatibility. + * + * @deprecated since 1.4, replaced by {@link org.apache.accumulo.core.iterators.user.WholeRowIterator} + */ +@Deprecated +public class WholeRowIterator extends org.apache.accumulo.core.iterators.user.WholeRowIterator { + +} http://git-wip-us.apache.org/repos/asf/accumulo/blob/4f2e6472/core/src/main/java/org/apache/accumulo/core/iterators/aggregation/Aggregator.java ---------------------------------------------------------------------- diff --git a/core/src/main/java/org/apache/accumulo/core/iterators/aggregation/Aggregator.java b/core/src/main/java/org/apache/accumulo/core/iterators/aggregation/Aggregator.java new file mode 100644 index 0000000..f9183dc --- /dev/null +++ b/core/src/main/java/org/apache/accumulo/core/iterators/aggregation/Aggregator.java @@ -0,0 +1,31 @@ +/* + * 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.accumulo.core.iterators.aggregation; + +import org.apache.accumulo.core.data.Value; + +/** + * @deprecated since 1.4, replaced by {@link org.apache.accumulo.core.iterators.Combiner} + */ +@Deprecated +public interface Aggregator { + void reset(); + + void collect(Value value); + + Value aggregate(); +} http://git-wip-us.apache.org/repos/asf/accumulo/blob/4f2e6472/core/src/main/java/org/apache/accumulo/core/iterators/aggregation/LongSummation.java ---------------------------------------------------------------------- diff --git a/core/src/main/java/org/apache/accumulo/core/iterators/aggregation/LongSummation.java b/core/src/main/java/org/apache/accumulo/core/iterators/aggregation/LongSummation.java new file mode 100644 index 0000000..7692ecb --- /dev/null +++ b/core/src/main/java/org/apache/accumulo/core/iterators/aggregation/LongSummation.java @@ -0,0 +1,76 @@ +/* + * 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.accumulo.core.iterators.aggregation; + +import java.io.IOException; + +import org.apache.accumulo.core.data.Value; +import org.slf4j.Logger; +import org.slf4j.LoggerFactory; + +/** + * @deprecated since 1.4, replaced by {@link org.apache.accumulo.core.iterators.user.SummingCombiner} with + * {@link org.apache.accumulo.core.iterators.LongCombiner.Type#FIXEDLEN} + */ +@Deprecated +public class LongSummation implements Aggregator { + private static final Logger log = LoggerFactory.getLogger(LongSummation.class); + long sum = 0; + + @Override + public Value aggregate() { + return new Value(longToBytes(sum)); + } + + @Override + public void collect(Value value) { + try { + sum += bytesToLong(value.get()); + } catch (IOException e) { + log.error(LongSummation.class.getSimpleName() + " trying to convert bytes to long, but byte array isn't length 8"); + } + } + + @Override + public void reset() { + sum = 0; + } + + public static long bytesToLong(byte[] b) throws IOException { + return bytesToLong(b, 0); + } + + public static long bytesToLong(byte[] b, int offset) throws IOException { + if (b.length < offset + 8) + throw new IOException("trying to convert to long, but byte array isn't long enough, wanted " + (offset + 8) + " found " + b.length); + return (((long) b[offset + 0] << 56) + ((long) (b[offset + 1] & 255) << 48) + ((long) (b[offset + 2] & 255) << 40) + ((long) (b[offset + 3] & 255) << 32) + + ((long) (b[offset + 4] & 255) << 24) + ((b[offset + 5] & 255) << 16) + ((b[offset + 6] & 255) << 8) + ((b[offset + 7] & 255) << 0)); + } + + public static byte[] longToBytes(long l) { + byte[] b = new byte[8]; + b[0] = (byte) (l >>> 56); + b[1] = (byte) (l >>> 48); + b[2] = (byte) (l >>> 40); + b[3] = (byte) (l >>> 32); + b[4] = (byte) (l >>> 24); + b[5] = (byte) (l >>> 16); + b[6] = (byte) (l >>> 8); + b[7] = (byte) (l >>> 0); + return b; + } +} http://git-wip-us.apache.org/repos/asf/accumulo/blob/4f2e6472/core/src/main/java/org/apache/accumulo/core/iterators/aggregation/NumArraySummation.java ---------------------------------------------------------------------- diff --git a/core/src/main/java/org/apache/accumulo/core/iterators/aggregation/NumArraySummation.java b/core/src/main/java/org/apache/accumulo/core/iterators/aggregation/NumArraySummation.java new file mode 100644 index 0000000..66cd2d5 --- /dev/null +++ b/core/src/main/java/org/apache/accumulo/core/iterators/aggregation/NumArraySummation.java @@ -0,0 +1,96 @@ +/* + * 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.accumulo.core.iterators.aggregation; + +import java.io.ByteArrayInputStream; +import java.io.ByteArrayOutputStream; +import java.io.DataInputStream; +import java.io.DataOutputStream; +import java.io.IOException; + +import org.apache.accumulo.core.data.Value; +import org.apache.hadoop.io.WritableUtils; + +/** + * @deprecated since 1.4, replaced by {@link org.apache.accumulo.core.iterators.user.SummingArrayCombiner} with + * {@link org.apache.accumulo.core.iterators.user.SummingArrayCombiner.Type#VARLEN} + */ +@Deprecated +public class NumArraySummation implements Aggregator { + long[] sum = new long[0]; + + @Override + public Value aggregate() { + try { + return new Value(NumArraySummation.longArrayToBytes(sum)); + } catch (IOException e) { + throw new RuntimeException(e); + } + } + + @Override + public void collect(Value value) { + long[] la; + try { + la = NumArraySummation.bytesToLongArray(value.get()); + } catch (IOException e) { + throw new RuntimeException(e); + } + + if (la.length > sum.length) { + for (int i = 0; i < sum.length; i++) { + la[i] = NumSummation.safeAdd(la[i], sum[i]); + } + sum = la; + } else { + for (int i = 0; i < la.length; i++) { + sum[i] = NumSummation.safeAdd(sum[i], la[i]); + } + } + } + + public static byte[] longArrayToBytes(long[] la) throws IOException { + ByteArrayOutputStream baos = new ByteArrayOutputStream(); + DataOutputStream dos = new DataOutputStream(baos); + + WritableUtils.writeVInt(dos, la.length); + for (int i = 0; i < la.length; i++) { + WritableUtils.writeVLong(dos, la[i]); + } + + return baos.toByteArray(); + } + + public static long[] bytesToLongArray(byte[] b) throws IOException { + DataInputStream dis = new DataInputStream(new ByteArrayInputStream(b)); + int len = WritableUtils.readVInt(dis); + + long[] la = new long[len]; + + for (int i = 0; i < len; i++) { + la[i] = WritableUtils.readVLong(dis); + } + + return la; + } + + @Override + public void reset() { + sum = new long[0]; + } + +}