3 * Licensed to the Apache Software Foundation (ASF) under one
4 * or more contributor license agreements. See the NOTICE file
5 * distributed with this work for additional information
6 * regarding copyright ownership. The ASF licenses this file
7 * to you under the Apache License, Version 2.0 (the
8 * "License"); you may not use this file except in compliance
9 * with the License. You may obtain a copy of the License at
11 * http://www.apache.org/licenses/LICENSE-2.0
13 * Unless required by applicable law or agreed to in writing, software
14 * distributed under the License is distributed on an "AS IS" BASIS,
15 * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
16 * See the License for the specific language governing permissions and
17 * limitations under the License.
19 package org
.apache
.hadoop
.hbase
.thrift
;
21 import java
.nio
.ByteBuffer
;
22 import java
.nio
.charset
.CharacterCodingException
;
23 import java
.nio
.charset
.Charset
;
24 import java
.nio
.charset
.CharsetDecoder
;
25 import java
.security
.PrivilegedExceptionAction
;
26 import java
.text
.NumberFormat
;
27 import java
.util
.ArrayList
;
28 import java
.util
.HashMap
;
29 import java
.util
.List
;
31 import java
.util
.SortedMap
;
32 import java
.util
.TreeMap
;
33 import javax
.security
.auth
.Subject
;
34 import javax
.security
.auth
.login
.AppConfigurationEntry
;
35 import javax
.security
.auth
.login
.Configuration
;
36 import javax
.security
.auth
.login
.LoginContext
;
37 import javax
.security
.sasl
.Sasl
;
38 import org
.apache
.hadoop
.hbase
.thrift
.generated
.AlreadyExists
;
39 import org
.apache
.hadoop
.hbase
.thrift
.generated
.ColumnDescriptor
;
40 import org
.apache
.hadoop
.hbase
.thrift
.generated
.Hbase
;
41 import org
.apache
.hadoop
.hbase
.thrift
.generated
.Mutation
;
42 import org
.apache
.hadoop
.hbase
.thrift
.generated
.TCell
;
43 import org
.apache
.hadoop
.hbase
.thrift
.generated
.TRowResult
;
44 import org
.apache
.hadoop
.hbase
.util
.Bytes
;
45 import org
.apache
.thrift
.protocol
.TBinaryProtocol
;
46 import org
.apache
.thrift
.protocol
.TProtocol
;
47 import org
.apache
.thrift
.transport
.TSaslClientTransport
;
48 import org
.apache
.thrift
.transport
.TSocket
;
49 import org
.apache
.thrift
.transport
.TTransport
;
50 import org
.apache
.yetus
.audience
.InterfaceAudience
;
51 import org
.slf4j
.Logger
;
52 import org
.slf4j
.LoggerFactory
;
55 * See the instructions under hbase-examples/README.txt
57 @InterfaceAudience.Private
58 public class DemoClient
{
59 private static final Logger LOG
= LoggerFactory
.getLogger(DemoClient
.class);
61 static protected int port
;
62 static protected String host
;
63 CharsetDecoder decoder
= null;
65 private static boolean secure
= false;
66 private static String serverPrincipal
= "hbase";
68 public static void main(String
[] args
) throws Exception
{
69 if (args
.length
< 2 || args
.length
> 4 || (args
.length
> 2 && !isBoolean(args
[2]))) {
70 System
.out
.println("Invalid arguments!");
71 System
.out
.println("Usage: DemoClient host port [secure=false [server-principal=hbase] ]");
76 port
= Integer
.parseInt(args
[1]);
79 if (args
.length
> 2) {
80 secure
= Boolean
.parseBoolean(args
[2]);
83 if (args
.length
== 4) {
84 serverPrincipal
= args
[3];
87 final DemoClient client
= new DemoClient();
88 Subject
.doAs(getSubject(),
89 new PrivilegedExceptionAction
<Void
>() {
91 public Void
run() throws Exception
{
98 private static boolean isBoolean(String s
){
99 return Boolean
.TRUE
.toString().equalsIgnoreCase(s
) ||
100 Boolean
.FALSE
.toString().equalsIgnoreCase(s
);
104 decoder
= Charset
.forName("UTF-8").newDecoder();
107 // Helper to translate byte[]'s to UTF8 strings
108 private String
utf8(byte[] buf
) {
110 return decoder
.decode(ByteBuffer
.wrap(buf
)).toString();
111 } catch (CharacterCodingException e
) {
112 return "[INVALID UTF-8]";
116 // Helper to translate strings to UTF8 bytes
117 private byte[] bytes(String s
) {
118 return Bytes
.toBytes(s
);
121 private void run() throws Exception
{
122 TTransport transport
= new TSocket(host
, port
);
124 Map
<String
, String
> saslProperties
= new HashMap
<>();
125 saslProperties
.put(Sasl
.QOP
, "auth-conf,auth-int,auth");
127 * The Thrift server the DemoClient is trying to connect to
128 * must have a matching principal, and support authentication.
130 * The HBase cluster must be secure, allow proxy user.
132 transport
= new TSaslClientTransport("GSSAPI", null,
133 serverPrincipal
, // Thrift server user name, should be an authorized proxy user.
134 host
, // Thrift server domain
135 saslProperties
, null, transport
);
140 TProtocol protocol
= new TBinaryProtocol(transport
, true, true);
141 Hbase
.Client client
= new Hbase
.Client(protocol
);
143 byte[] t
= bytes("demo_table");
145 // Scan all tables, look for the demo table and delete it.
146 System
.out
.println("scanning tables...");
148 for (ByteBuffer name
: client
.getTableNames()) {
149 System
.out
.println(" found: " + utf8(name
.array()));
151 if (utf8(name
.array()).equals(utf8(t
))) {
152 if (client
.isTableEnabled(name
)) {
153 System
.out
.println(" disabling table: " + utf8(name
.array()));
154 client
.disableTable(name
);
157 System
.out
.println(" deleting table: " + utf8(name
.array()));
158 client
.deleteTable(name
);
162 // Create the demo table with two column families, entry: and unused:
163 ArrayList
<ColumnDescriptor
> columns
= new ArrayList
<>(2);
164 ColumnDescriptor col
;
165 col
= new ColumnDescriptor();
166 col
.name
= ByteBuffer
.wrap(bytes("entry:"));
167 col
.timeToLive
= Integer
.MAX_VALUE
;
168 col
.maxVersions
= 10;
170 col
= new ColumnDescriptor();
171 col
.name
= ByteBuffer
.wrap(bytes("unused:"));
172 col
.timeToLive
= Integer
.MAX_VALUE
;
175 System
.out
.println("creating table: " + utf8(t
));
178 client
.createTable(ByteBuffer
.wrap(t
), columns
);
179 } catch (AlreadyExists ae
) {
180 System
.out
.println("WARN: " + ae
.message
);
183 System
.out
.println("column families in " + utf8(t
) + ": ");
184 Map
<ByteBuffer
, ColumnDescriptor
> columnMap
= client
.getColumnDescriptors(ByteBuffer
.wrap(t
));
186 for (ColumnDescriptor col2
: columnMap
.values()) {
187 System
.out
.println(" column: " + utf8(col2
.name
.array()) + ", maxVer: " + col2
.maxVersions
);
190 Map
<ByteBuffer
, ByteBuffer
> dummyAttributes
= null;
191 boolean writeToWal
= false;
193 // Test UTF-8 handling
194 byte[] invalid
= {(byte) 'f', (byte) 'o', (byte) 'o', (byte) '-',
195 (byte) 0xfc, (byte) 0xa1, (byte) 0xa1, (byte) 0xa1, (byte) 0xa1};
196 byte[] valid
= {(byte) 'f', (byte) 'o', (byte) 'o', (byte) '-',
197 (byte) 0xE7, (byte) 0x94, (byte) 0x9F, (byte) 0xE3, (byte) 0x83,
198 (byte) 0x93, (byte) 0xE3, (byte) 0x83, (byte) 0xBC, (byte) 0xE3,
199 (byte) 0x83, (byte) 0xAB};
201 ArrayList
<Mutation
> mutations
;
202 // non-utf8 is fine for data
203 mutations
= new ArrayList
<>(1);
204 mutations
.add(new Mutation(false, ByteBuffer
.wrap(bytes("entry:foo")),
205 ByteBuffer
.wrap(invalid
), writeToWal
));
206 client
.mutateRow(ByteBuffer
.wrap(t
), ByteBuffer
.wrap(bytes("foo")),
207 mutations
, dummyAttributes
);
209 // this row name is valid utf8
210 mutations
= new ArrayList
<>(1);
211 mutations
.add(new Mutation(false, ByteBuffer
.wrap(bytes("entry:foo")),
212 ByteBuffer
.wrap(valid
), writeToWal
));
213 client
.mutateRow(ByteBuffer
.wrap(t
), ByteBuffer
.wrap(valid
), mutations
, dummyAttributes
);
215 // non-utf8 is now allowed in row names because HBase stores values as binary
216 mutations
= new ArrayList
<>(1);
217 mutations
.add(new Mutation(false, ByteBuffer
.wrap(bytes("entry:foo")),
218 ByteBuffer
.wrap(invalid
), writeToWal
));
219 client
.mutateRow(ByteBuffer
.wrap(t
), ByteBuffer
.wrap(invalid
), mutations
, dummyAttributes
);
221 // Run a scanner on the rows we just created
222 ArrayList
<ByteBuffer
> columnNames
= new ArrayList
<>();
223 columnNames
.add(ByteBuffer
.wrap(bytes("entry:")));
225 System
.out
.println("Starting scanner...");
226 int scanner
= client
.scannerOpen(ByteBuffer
.wrap(t
), ByteBuffer
.wrap(bytes("")), columnNames
,
230 List
<TRowResult
> entry
= client
.scannerGet(scanner
);
232 if (entry
.isEmpty()) {
239 // Run some operations on a bunch of rows
240 for (int i
= 100; i
>= 0; --i
) {
241 // format row keys as "00000" to "00100"
242 NumberFormat nf
= NumberFormat
.getInstance();
243 nf
.setMinimumIntegerDigits(5);
244 nf
.setGroupingUsed(false);
245 byte[] row
= bytes(nf
.format(i
));
247 mutations
= new ArrayList
<>(1);
248 mutations
.add(new Mutation(false, ByteBuffer
.wrap(bytes("unused:")),
249 ByteBuffer
.wrap(bytes("DELETE_ME")), writeToWal
));
250 client
.mutateRow(ByteBuffer
.wrap(t
), ByteBuffer
.wrap(row
), mutations
, dummyAttributes
);
251 printRow(client
.getRow(ByteBuffer
.wrap(t
), ByteBuffer
.wrap(row
), dummyAttributes
));
252 client
.deleteAllRow(ByteBuffer
.wrap(t
), ByteBuffer
.wrap(row
), dummyAttributes
);
254 // sleep to force later timestamp
257 } catch (InterruptedException e
) {
261 mutations
= new ArrayList
<>(2);
262 mutations
.add(new Mutation(false, ByteBuffer
.wrap(bytes("entry:num")),
263 ByteBuffer
.wrap(bytes("0")), writeToWal
));
264 mutations
.add(new Mutation(false, ByteBuffer
.wrap(bytes("entry:foo")),
265 ByteBuffer
.wrap(bytes("FOO")), writeToWal
));
266 client
.mutateRow(ByteBuffer
.wrap(t
), ByteBuffer
.wrap(row
), mutations
, dummyAttributes
);
267 printRow(client
.getRow(ByteBuffer
.wrap(t
), ByteBuffer
.wrap(row
), dummyAttributes
));
270 mutations
= new ArrayList
<>(2);
272 m
.column
= ByteBuffer
.wrap(bytes("entry:foo"));
276 m
.column
= ByteBuffer
.wrap(bytes("entry:num"));
277 m
.value
= ByteBuffer
.wrap(bytes("-1"));
279 client
.mutateRow(ByteBuffer
.wrap(t
), ByteBuffer
.wrap(row
), mutations
, dummyAttributes
);
280 printRow(client
.getRow(ByteBuffer
.wrap(t
), ByteBuffer
.wrap(row
), dummyAttributes
));
282 mutations
= new ArrayList
<>();
283 mutations
.add(new Mutation(false, ByteBuffer
.wrap(bytes("entry:num")),
284 ByteBuffer
.wrap(bytes(Integer
.toString(i
))), writeToWal
));
285 mutations
.add(new Mutation(false, ByteBuffer
.wrap(bytes("entry:sqr")),
286 ByteBuffer
.wrap(bytes(Integer
.toString(i
* i
))), writeToWal
));
287 client
.mutateRow(ByteBuffer
.wrap(t
), ByteBuffer
.wrap(row
), mutations
, dummyAttributes
);
288 printRow(client
.getRow(ByteBuffer
.wrap(t
), ByteBuffer
.wrap(row
), dummyAttributes
));
290 // sleep to force later timestamp
293 } catch (InterruptedException e
) {
299 m
.column
= ByteBuffer
.wrap(bytes("entry:num"));
300 m
.value
= ByteBuffer
.wrap(bytes("-999"));
303 m
.column
= ByteBuffer
.wrap(bytes("entry:sqr"));
305 client
.mutateRowTs(ByteBuffer
.wrap(t
), ByteBuffer
.wrap(row
), mutations
, 1,
306 dummyAttributes
); // shouldn't override latest
307 printRow(client
.getRow(ByteBuffer
.wrap(t
), ByteBuffer
.wrap(row
), dummyAttributes
));
309 List
<TCell
> versions
= client
.getVer(ByteBuffer
.wrap(t
), ByteBuffer
.wrap(row
),
310 ByteBuffer
.wrap(bytes("entry:num")), 10, dummyAttributes
);
311 printVersions(ByteBuffer
.wrap(row
), versions
);
313 if (versions
.isEmpty()) {
314 System
.out
.println("FATAL: wrong # of versions");
318 List
<TCell
> result
= client
.get(ByteBuffer
.wrap(t
), ByteBuffer
.wrap(row
),
319 ByteBuffer
.wrap(bytes("entry:foo")), dummyAttributes
);
321 if (!result
.isEmpty()) {
322 System
.out
.println("FATAL: shouldn't get here");
326 System
.out
.println("");
329 // scan all rows/columnNames
332 for (ColumnDescriptor col2
: client
.getColumnDescriptors(ByteBuffer
.wrap(t
)).values()) {
333 System
.out
.println("column with name: " + new String(col2
.name
.array()));
334 System
.out
.println(col2
.toString());
336 columnNames
.add(col2
.name
);
339 System
.out
.println("Starting scanner...");
340 scanner
= client
.scannerOpenWithStop(ByteBuffer
.wrap(t
), ByteBuffer
.wrap(bytes("00020")),
341 ByteBuffer
.wrap(bytes("00040")), columnNames
, dummyAttributes
);
344 List
<TRowResult
> entry
= client
.scannerGet(scanner
);
346 if (entry
.isEmpty()) {
347 System
.out
.println("Scanner finished");
357 private void printVersions(ByteBuffer row
, List
<TCell
> versions
) {
358 StringBuilder rowStr
= new StringBuilder();
360 for (TCell cell
: versions
) {
361 rowStr
.append(utf8(cell
.value
.array()));
365 System
.out
.println("row: " + utf8(row
.array()) + ", values: " + rowStr
);
368 private void printRow(TRowResult rowResult
) {
369 // copy values into a TreeMap to get them in sorted order
370 TreeMap
<String
, TCell
> sorted
= new TreeMap
<>();
372 for (Map
.Entry
<ByteBuffer
, TCell
> column
: rowResult
.columns
.entrySet()) {
373 sorted
.put(utf8(column
.getKey().array()), column
.getValue());
376 StringBuilder rowStr
= new StringBuilder();
378 for (SortedMap
.Entry
<String
, TCell
> entry
: sorted
.entrySet()) {
379 rowStr
.append(entry
.getKey());
380 rowStr
.append(" => ");
381 rowStr
.append(utf8(entry
.getValue().value
.array()));
385 System
.out
.println("row: " + utf8(rowResult
.row
.array()) + ", cols: " + rowStr
);
388 private void printRow(List
<TRowResult
> rows
) {
389 for (TRowResult rowResult
: rows
) {
394 static Subject
getSubject() throws Exception
{
396 return new Subject();
400 * To authenticate the DemoClient, kinit should be invoked ahead.
401 * Here we try to get the Kerberos credential from the ticket cache.
403 LoginContext context
= new LoginContext("", new Subject(), null,
404 new Configuration() {
406 public AppConfigurationEntry
[] getAppConfigurationEntry(String name
) {
407 Map
<String
, String
> options
= new HashMap
<>();
408 options
.put("useKeyTab", "false");
409 options
.put("storeKey", "false");
410 options
.put("doNotPrompt", "true");
411 options
.put("useTicketCache", "true");
412 options
.put("renewTGT", "true");
413 options
.put("refreshKrb5Config", "true");
414 options
.put("isInitiator", "true");
415 String ticketCache
= System
.getenv("KRB5CCNAME");
417 if (ticketCache
!= null) {
418 options
.put("ticketCache", ticketCache
);
421 options
.put("debug", "true");
423 return new AppConfigurationEntry
[]{
424 new AppConfigurationEntry("com.sun.security.auth.module.Krb5LoginModule",
425 AppConfigurationEntry
.LoginModuleControlFlag
.REQUIRED
,
431 return context
.getSubject();