Skip to content

Commit

Permalink
Merge remote-tracking branch 'upstream/3.1' into removeProbReports-sq…
Browse files Browse the repository at this point in the history
…uashed
  • Loading branch information
DomGarguilo committed Dec 2, 2024
2 parents 5820cd3 + dd2d40f commit 326f771
Show file tree
Hide file tree
Showing 53 changed files with 334 additions and 252 deletions.
Original file line number Diff line number Diff line change
Expand Up @@ -510,8 +510,7 @@ public String getRootTabletLocation() {
*/
public List<String> getManagerLocations() {
ensureOpen();
var zLockManagerPath =
ServiceLock.path(Constants.ZROOT + "/" + getInstanceID() + Constants.ZMANAGER_LOCK);
var zLockManagerPath = ServiceLock.path(getZooKeeperRoot() + Constants.ZMANAGER_LOCK);

Timer timer = null;

Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -20,37 +20,54 @@

import static org.easymock.EasyMock.createMock;
import static org.easymock.EasyMock.expect;
import static org.easymock.EasyMock.expectLastCall;
import static org.easymock.EasyMock.replay;
import static org.easymock.EasyMock.reset;
import static org.easymock.EasyMock.verify;

import java.util.UUID;

import org.apache.accumulo.core.Constants;
import org.apache.accumulo.core.clientImpl.TabletLocatorImpl.TabletServerLockChecker;
import org.apache.accumulo.core.data.InstanceId;
import org.apache.accumulo.core.fate.zookeeper.ZooCache;
import org.apache.accumulo.core.fate.zookeeper.ZooUtil;
import org.junit.jupiter.api.AfterEach;
import org.junit.jupiter.api.BeforeEach;
import org.junit.jupiter.api.Test;

public class RootTabletLocatorTest {

private ClientContext context;
private TabletServerLockChecker lockChecker;
private ZooCache zc;
private RootTabletLocator rtl;

@BeforeEach
public void setUp() {
context = createMock(ClientContext.class);
expect(context.getZooKeeperRoot()).andReturn("/accumulo/iid").anyTimes();
var instanceId = InstanceId.of(UUID.randomUUID());
zc = createMock(ZooCache.class);
context = createMock(ClientContext.class);
expect(context.getZooKeeperRoot()).andReturn(ZooUtil.getRoot(instanceId)).anyTimes();
expect(context.getZooCache()).andReturn(zc).anyTimes();
replay(context);
lockChecker = createMock(TabletServerLockChecker.class);
rtl = new RootTabletLocator(lockChecker);
replay(context, zc, lockChecker);
}

@AfterEach
public void tearDown() {
verify(context, zc, lockChecker);
}

@Test
public void testInvalidateCache_Server() {
var rtl = new RootTabletLocator(lockChecker);

verify(zc);
reset(zc);
zc.clear(context.getZooKeeperRoot() + Constants.ZTSERVERS + "/server");
expectLastCall().once();
replay(zc);

rtl.invalidateCache(context, "server");
verify(zc);
}
}
Original file line number Diff line number Diff line change
Expand Up @@ -20,34 +20,51 @@

import static org.easymock.EasyMock.createMock;
import static org.easymock.EasyMock.expect;
import static org.easymock.EasyMock.expectLastCall;
import static org.easymock.EasyMock.replay;
import static org.easymock.EasyMock.reset;
import static org.easymock.EasyMock.verify;

import java.util.UUID;

import org.apache.accumulo.core.Constants;
import org.apache.accumulo.core.data.InstanceId;
import org.apache.accumulo.core.fate.zookeeper.ZooCache;
import org.apache.accumulo.core.fate.zookeeper.ZooUtil;
import org.junit.jupiter.api.AfterEach;
import org.junit.jupiter.api.BeforeEach;
import org.junit.jupiter.api.Test;

public class ZookeeperLockCheckerTest {

private ClientContext context;
private ZooCache zc;
private ZookeeperLockChecker zklc;

@BeforeEach
public void setUp() {
context = createMock(ClientContext.class);
expect(context.getZooKeeperRoot()).andReturn("/accumulo/iid").anyTimes();
var instanceId = InstanceId.of(UUID.randomUUID());
zc = createMock(ZooCache.class);
context = createMock(ClientContext.class);
expect(context.getZooKeeperRoot()).andReturn(ZooUtil.getRoot(instanceId)).anyTimes();
expect(context.getZooCache()).andReturn(zc).anyTimes();
replay(context);
zklc = new ZookeeperLockChecker(context);
replay(context, zc);
}

@AfterEach
public void tearDown() {
verify(context, zc);
}

@Test
public void testInvalidateCache() {
var zklc = new ZookeeperLockChecker(context);

verify(zc);
reset(zc);
zc.clear(context.getZooKeeperRoot() + Constants.ZTSERVERS + "/server");
expectLastCall().once();
replay(zc);

zklc.invalidateCache("server");
verify(zc);
}
}
Original file line number Diff line number Diff line change
Expand Up @@ -26,8 +26,10 @@
import java.io.File;
import java.util.Map;

import org.apache.accumulo.core.Constants;
import org.apache.accumulo.core.client.Accumulo;
import org.apache.accumulo.core.client.AccumuloClient;
import org.apache.accumulo.core.fate.zookeeper.ZooUtil;
import org.apache.commons.io.FileUtils;
import org.apache.curator.framework.CuratorFramework;
import org.apache.curator.framework.CuratorFrameworkFactory;
Expand Down Expand Up @@ -74,8 +76,9 @@ public void canConnectViaExistingZooKeeper() throws Exception {
Map<String,String> tableIds = client.tableOperations().tableIdMap();
assertTrue(tableIds.containsKey(tableName));

String zkTablePath = String.format("/accumulo/%s/tables/%s/name",
client.instanceOperations().getInstanceId().canonical(), tableIds.get(tableName));
String zkTablePath = String.format("%s%s/%s/name",
ZooUtil.getRoot(client.instanceOperations().getInstanceId()), Constants.ZTABLES,
tableIds.get(tableName));
try (CuratorFramework curatorClient =
CuratorFrameworkFactory.newClient(zooKeeper.getConnectString(), new RetryOneTime(1))) {
curatorClient.start();
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -52,6 +52,7 @@
import org.apache.accumulo.core.data.TableId;
import org.apache.accumulo.core.fate.zookeeper.ZooReader;
import org.apache.accumulo.core.fate.zookeeper.ZooReaderWriter;
import org.apache.accumulo.core.fate.zookeeper.ZooUtil;
import org.apache.accumulo.core.metadata.schema.Ample;
import org.apache.accumulo.core.metrics.MetricsInfo;
import org.apache.accumulo.core.rpc.SslConnectionParams;
Expand Down Expand Up @@ -118,7 +119,7 @@ private ServerContext(ServerInfo info) {
serverDirs = info.getServerDirs();

propStore = memoize(() -> ZooPropStore.initialize(getInstanceID(), getZooReaderWriter()));
zkUserPath = memoize(() -> Constants.ZROOT + "/" + getInstanceID() + Constants.ZUSERS);
zkUserPath = memoize(() -> ZooUtil.getRoot(getInstanceID()) + Constants.ZUSERS);

tableManager = memoize(() -> new TableManager(this));
nameAllocator = memoize(() -> new UniqueNameAllocator(this));
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -36,6 +36,7 @@
import org.apache.accumulo.core.data.InstanceId;
import org.apache.accumulo.core.fate.zookeeper.ZooCache;
import org.apache.accumulo.core.fate.zookeeper.ZooCacheFactory;
import org.apache.accumulo.core.fate.zookeeper.ZooUtil;
import org.apache.accumulo.core.singletons.SingletonManager;
import org.apache.accumulo.core.singletons.SingletonManager.Mode;
import org.apache.accumulo.server.fs.VolumeManager;
Expand Down Expand Up @@ -79,7 +80,7 @@ public class ServerInfo implements ClientInfo {
+ "Run \"accumulo org.apache.accumulo.server.util.ListInstances\" to see a list.");
}
instanceID = InstanceId.of(new String(iidb, UTF_8));
if (zooCache.get(Constants.ZROOT + "/" + instanceID) == null) {
if (zooCache.get(ZooUtil.getRoot(instanceID)) == null) {
if (instanceName == null) {
throw new IllegalStateException(
"Instance id " + instanceID + " does not exist in zookeeper");
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -71,7 +71,7 @@ void initializeConfig(final InstanceId instanceId, final ZooReaderWriter zoo) {
zoo.putPersistentData(Constants.ZROOT, new byte[0], ZooUtil.NodeExistsPolicy.SKIP,
ZooDefs.Ids.OPEN_ACL_UNSAFE);

String zkInstanceRoot = Constants.ZROOT + "/" + instanceId;
String zkInstanceRoot = ZooUtil.getRoot(instanceId);
zoo.putPersistentData(zkInstanceRoot, EMPTY_BYTE_ARRAY, ZooUtil.NodeExistsPolicy.SKIP);
var sysPropPath = SystemPropKey.of(instanceId).getPath();
VersionedProperties vProps = new VersionedProperties();
Expand Down Expand Up @@ -109,7 +109,7 @@ void initialize(final ServerContext context, final boolean clearInstanceName,
ZooUtil.NodeExistsPolicy.FAIL);

// setup the instance
String zkInstanceRoot = Constants.ZROOT + "/" + instanceId;
String zkInstanceRoot = context.getZooKeeperRoot();
zoo.putPersistentData(zkInstanceRoot + Constants.ZTABLES, Constants.ZTABLES_INITIAL_ID,
ZooUtil.NodeExistsPolicy.FAIL);
zoo.putPersistentData(zkInstanceRoot + Constants.ZNAMESPACES,
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -27,12 +27,12 @@
import java.util.Set;
import java.util.TreeSet;

import org.apache.accumulo.core.Constants;
import org.apache.accumulo.core.client.AccumuloSecurityException;
import org.apache.accumulo.core.client.NamespaceNotFoundException;
import org.apache.accumulo.core.client.TableNotFoundException;
import org.apache.accumulo.core.clientImpl.Namespace;
import org.apache.accumulo.core.clientImpl.thrift.SecurityErrorCode;
import org.apache.accumulo.core.data.InstanceId;
import org.apache.accumulo.core.data.NamespaceId;
import org.apache.accumulo.core.data.TableId;
import org.apache.accumulo.core.fate.zookeeper.ZooCache;
Expand Down Expand Up @@ -66,10 +66,9 @@ public class ZKPermHandler implements PermissionHandler {
public void initialize(ServerContext context) {
zooCache = new ZooCache(context.getZooReader(), null);
zoo = context.getZooReaderWriter();
InstanceId instanceId = context.getInstanceID();
zkUserPath = context.zkUserPath();
ZKTablePath = ZKSecurityTool.getInstancePath(instanceId) + "/tables";
ZKNamespacePath = ZKSecurityTool.getInstancePath(instanceId) + "/namespaces";
ZKTablePath = context.getZooKeeperRoot() + Constants.ZTABLES;
ZKNamespacePath = context.getZooKeeperRoot() + Constants.ZNAMESPACES;
}

@Override
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -31,9 +31,7 @@
import java.util.HashSet;
import java.util.Set;

import org.apache.accumulo.core.Constants;
import org.apache.accumulo.core.client.AccumuloException;
import org.apache.accumulo.core.data.InstanceId;
import org.apache.accumulo.core.security.Authorizations;
import org.apache.accumulo.core.security.NamespacePermission;
import org.apache.accumulo.core.security.SystemPermission;
Expand Down Expand Up @@ -191,7 +189,4 @@ public static Set<NamespacePermission> convertNamespacePermissions(byte[] namesp
return toReturn;
}

public static String getInstancePath(InstanceId instanceId) {
return Constants.ZROOT + "/" + instanceId;
}
}
Original file line number Diff line number Diff line change
Expand Up @@ -37,6 +37,7 @@
import org.apache.accumulo.core.data.TableId;
import org.apache.accumulo.core.fate.zookeeper.ZooCache;
import org.apache.accumulo.core.fate.zookeeper.ZooReaderWriter;
import org.apache.accumulo.core.fate.zookeeper.ZooUtil;
import org.apache.accumulo.core.fate.zookeeper.ZooUtil.NodeExistsPolicy;
import org.apache.accumulo.core.fate.zookeeper.ZooUtil.NodeMissingPolicy;
import org.apache.accumulo.core.manager.state.tables.TableState;
Expand Down Expand Up @@ -77,7 +78,7 @@ public static void prepareNewNamespaceState(final ServerContext context, Namespa
final InstanceId instanceId = context.getInstanceID();
log.debug("Creating ZooKeeper entries for new namespace {} (ID: {})", namespace, namespaceId);
context.getZooReaderWriter().putPersistentData(
Constants.ZROOT + "/" + instanceId + Constants.ZNAMESPACES + "/" + namespaceId, new byte[0],
context.getZooKeeperRoot() + Constants.ZNAMESPACES + "/" + namespaceId, new byte[0],
existsPolicy);
var propKey = NamespacePropKey.of(instanceId, namespaceId);
if (!propStore.exists(propKey)) {
Expand All @@ -94,7 +95,7 @@ public static void prepareNewTableState(ZooReaderWriter zoo, PropStore propStore
tableName, tableId, namespaceId);
Pair<String,String> qualifiedTableName = TableNameUtil.qualify(tableName);
tableName = qualifiedTableName.getSecond();
String zTablePath = Constants.ZROOT + "/" + instanceId + Constants.ZTABLES + "/" + tableId;
String zTablePath = ZooUtil.getRoot(instanceId) + Constants.ZTABLES + "/" + tableId;
zoo.putPersistentData(zTablePath, new byte[0], existsPolicy);
zoo.putPersistentData(zTablePath + Constants.ZTABLE_NAMESPACE,
namespaceId.canonical().getBytes(UTF_8), existsPolicy);
Expand Down Expand Up @@ -220,10 +221,10 @@ public void cloneTable(TableId srcTableId, TableId tableId, String tableName,
prepareNewTableState(zoo, context.getPropStore(), instanceID, tableId, namespaceId, tableName,
TableState.NEW, NodeExistsPolicy.OVERWRITE);

String srcTablePath = Constants.ZROOT + "/" + instanceID + Constants.ZTABLES + "/" + srcTableId
+ Constants.ZCONFIG;
String srcTablePath =
context.getZooKeeperRoot() + Constants.ZTABLES + "/" + srcTableId + Constants.ZCONFIG;
String newTablePath =
Constants.ZROOT + "/" + instanceID + Constants.ZTABLES + "/" + tableId + Constants.ZCONFIG;
context.getZooKeeperRoot() + Constants.ZTABLES + "/" + tableId + Constants.ZCONFIG;
zoo.recursiveCopyPersistentOverwrite(srcTablePath, newTablePath);

PropUtil.setProperties(context, TablePropKey.of(context, tableId), propertiesToSet);
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -50,7 +50,7 @@ public class UniqueNameAllocator {

public UniqueNameAllocator(ServerContext context) {
this.context = context;
nextNamePath = Constants.ZROOT + "/" + context.getInstanceID() + Constants.ZNEXT_FILE;
nextNamePath = context.getZooKeeperRoot() + Constants.ZNEXT_FILE;
}

public synchronized String getNextName() {
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -26,6 +26,7 @@
import java.util.List;
import java.util.UUID;

import org.apache.accumulo.core.Constants;
import org.apache.accumulo.core.conf.AccumuloConfiguration;
import org.apache.accumulo.core.data.InstanceId;
import org.apache.accumulo.core.fate.zookeeper.ZooReader;
Expand Down Expand Up @@ -145,7 +146,7 @@ private static void rewriteZooKeeperInstance(final ServerContext context,
}
}
});
String path = "/accumulo/instances/" + context.getInstanceName();
String path = Constants.ZROOT + Constants.ZINSTANCES + "/" + context.getInstanceName();
orig.recursiveDelete(path, NodeMissingPolicy.SKIP);
new_.putPersistentData(path, newInstanceId.canonical().getBytes(UTF_8),
NodeExistsPolicy.OVERWRITE);
Expand Down Expand Up @@ -201,6 +202,6 @@ private static void checkHdfsAccessPermissions(FileStatus stat, FsAction mode) t

private static void deleteInstance(ServerContext context, String oldPass) throws Exception {
ZooReaderWriter orig = context.getZooReader().asWriter(oldPass);
orig.recursiveDelete("/accumulo/" + context.getInstanceID(), NodeMissingPolicy.SKIP);
orig.recursiveDelete(context.getZooKeeperRoot(), NodeMissingPolicy.SKIP);
}
}
Original file line number Diff line number Diff line change
Expand Up @@ -35,6 +35,7 @@
import org.apache.accumulo.core.data.InstanceId;
import org.apache.accumulo.core.fate.zookeeper.ZooCache;
import org.apache.accumulo.core.fate.zookeeper.ZooReader;
import org.apache.accumulo.core.fate.zookeeper.ZooUtil;
import org.apache.accumulo.core.lock.ServiceLock;
import org.apache.accumulo.core.lock.ServiceLockData;
import org.apache.accumulo.core.lock.ServiceLockData.ThriftService;
Expand Down Expand Up @@ -165,8 +166,7 @@ private static String getManager(ZooCache cache, InstanceId iid, boolean printEr
}

try {
var zLockManagerPath =
ServiceLock.path(Constants.ZROOT + "/" + iid + Constants.ZMANAGER_LOCK);
var zLockManagerPath = ServiceLock.path(ZooUtil.getRoot(iid) + Constants.ZMANAGER_LOCK);
Optional<ServiceLockData> sld = ServiceLock.getLockData(cache, zLockManagerPath, null);
if (sld.isEmpty()) {
return null;
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -69,7 +69,7 @@ public void execute(final String[] args) throws Exception {
}
System.out.println("The accumulo instance id is " + context.getInstanceID());
if (!opts.servers.contains("/")) {
opts.servers += "/accumulo/" + context.getInstanceID();
opts.servers += context.getZooKeeperRoot();
}
org.apache.zookeeper.ZooKeeperMain
.main(new String[] {"-server", opts.servers, "-timeout", "" + (opts.timeout * 1000)});
Expand Down
Loading

0 comments on commit 326f771

Please sign in to comment.