22
33import com .clickhouse .client .api .ClientException ;
44import com .clickhouse .client .api .data_formats .ClickHouseBinaryFormatReader ;
5+ import com .clickhouse .client .api .internal .SerializerUtils ;
56import com .clickhouse .client .api .metadata .TableSchema ;
67import com .clickhouse .client .api .query .NullValueException ;
78import com .clickhouse .client .api .query .POJOSetter ;
3738import java .util .UUID ;
3839import java .util .concurrent .ConcurrentHashMap ;
3940import java .util .concurrent .atomic .AtomicBoolean ;
41+ import java .util .function .Function ;
4042
4143public abstract class AbstractBinaryFormatReader implements ClickHouseBinaryFormatReader {
4244
@@ -52,6 +54,8 @@ public abstract class AbstractBinaryFormatReader implements ClickHouseBinaryForm
5254
5355 private ClickHouseColumn [] columns ;
5456
57+ private Map [] convertions ;
58+
5559 private volatile boolean hasNext = true ;
5660
5761
@@ -61,14 +65,16 @@ protected AbstractBinaryFormatReader(InputStream inputStream, QuerySettings quer
6165 BinaryStreamReader .ByteBufferAllocator byteBufferAllocator ) {
6266 this .input = inputStream ;
6367 this .settings = querySettings == null ? Collections .emptyMap () : new HashMap <>(querySettings .getAllSettings ());
64- boolean useServerTimeZone = (boolean ) this .settings .get (ClickHouseClientOption .USE_SERVER_TIME_ZONE .getKey ());
65- TimeZone timeZone = useServerTimeZone ? querySettings .getServerTimeZone () :
68+ Boolean useServerTimeZone = (Boolean ) this .settings .get (ClickHouseClientOption .USE_SERVER_TIME_ZONE .getKey ());
69+ TimeZone timeZone = useServerTimeZone == Boolean . TRUE && querySettings != null ? querySettings .getServerTimeZone () :
6670 (TimeZone ) this .settings .get (ClickHouseClientOption .USE_TIME_ZONE .getKey ());
6771 if (timeZone == null ) {
6872 throw new ClientException ("Time zone is not set. (useServerTimezone:" + useServerTimeZone + ")" );
6973 }
7074 this .binaryStreamReader = new BinaryStreamReader (inputStream , timeZone , LOG , byteBufferAllocator );
71- setSchema (schema );
75+ if (schema != null ) {
76+ setSchema (schema );
77+ }
7278 }
7379
7480 protected Map <String , Object > currentRecord = new ConcurrentHashMap <>();
@@ -219,9 +225,49 @@ protected void endReached() {
219225
220226 protected void setSchema (TableSchema schema ) {
221227 this .schema = schema ;
222- if (schema != null ) {
223- columns = schema .getColumns ().toArray (new ClickHouseColumn [0 ]);
228+ this .columns = schema .getColumns ().toArray (new ClickHouseColumn [0 ]);
229+ this .convertions = new Map [columns .length ];
230+
231+ for (int i = 0 ; i < columns .length ; i ++) {
232+ ClickHouseColumn column = columns [i ];
233+
234+ Map <NumberType , Function <Number , ?>> converters = new HashMap <>();
235+ switch (column .getDataType ()) {
236+ case Int8 :
237+ case Int16 :
238+ case UInt8 :
239+ case Int32 :
240+ case UInt16 :
241+ case Int64 :
242+ case UInt32 :
243+ case Int128 :
244+ case UInt64 :
245+ case Int256 :
246+ case UInt128 :
247+ case UInt256 :
248+ case Float32 :
249+ case Float64 :
250+ case Decimal :
251+ case Decimal32 :
252+ case Decimal64 :
253+ case Decimal128 :
254+ case Decimal256 :
255+ case Bool :
256+ converters .put (NumberType .Byte , SerializerUtils .NumberConverter ::toByte );
257+ converters .put (NumberType .Short , SerializerUtils .NumberConverter ::toShort );
258+ converters .put (NumberType .Int , SerializerUtils .NumberConverter ::toInt );
259+ converters .put (NumberType .Long , SerializerUtils .NumberConverter ::toLong );
260+ converters .put (NumberType .BigInteger , SerializerUtils .NumberConverter ::toBigInteger );
261+ converters .put (NumberType .BigDecimal , SerializerUtils .NumberConverter ::toBigDecimal );
262+ converters .put (NumberType .Float , SerializerUtils .NumberConverter ::toFloat );
263+ converters .put (NumberType .Double , SerializerUtils .NumberConverter ::toDouble );
264+ converters .put (NumberType .Boolean , SerializerUtils ::convertToBoolean );
265+ break ;
266+ }
267+
268+ this .convertions [i ] = converters ;
224269 }
270+
225271 }
226272
227273 @ Override
@@ -252,65 +298,65 @@ public String getString(int index) {
252298 return value .toString ();
253299 }
254300
255- private <T > T readPrimitiveValue (String colName , String typeName ) {
256- Object value = readValue (colName );
257- if (value == null ) {
258- throw new NullValueException ("Column '" + colName + "' has null value and it cannot be cast to " + typeName );
259- }
260- return (T ) value ;
261- }
262-
263- private <T > T readPrimitiveValue (int colIndex , String typeName ) {
264- Object value = readValue (colIndex );
265- if (value == null ) {
266- throw new NullValueException ("Column at index = " + colIndex + " has null value and it cannot be cast to " + typeName );
301+ private <T > T readNumberValue (String colName , NumberType targetType ) {
302+ int colIndex = schema .nameToIndex (colName );
303+ Function <Number , Number > converter = (Function <Number , Number >) convertions [colIndex ].get (targetType );
304+ if (converter != null ) {
305+ Number value = readValue (colName );
306+ if (value == null ) {
307+ throw new NullValueException ("Column " + colName + " has null value and it cannot be cast to " +
308+ targetType .getTypeName ());
309+ }
310+ return (T ) converter .apply (value );
311+ } else {
312+ throw new ClientException ("Column " + colName + " " + columns [colIndex ].getDataType ().name () +
313+ " cannot be converted to " + targetType .getTypeName ());
267314 }
268- return (T ) value ;
269315 }
270316
271317 @ Override
272318 public byte getByte (String colName ) {
273- return readPrimitiveValue (colName , "byte" );
319+ return readNumberValue (colName , NumberType . Byte );
274320 }
275321
276322 @ Override
277323 public short getShort (String colName ) {
278- return readPrimitiveValue (colName , "short" );
324+ return readNumberValue (colName , NumberType . Short );
279325 }
280326
281327 @ Override
282328 public int getInteger (String colName ) {
283- return readPrimitiveValue (colName , "int" );
329+ return readNumberValue (colName , NumberType . Int );
284330 }
285331
286332 @ Override
287333 public long getLong (String colName ) {
288- return readPrimitiveValue (colName , "long" );
334+ return readNumberValue (colName , NumberType . Long );
289335 }
290336
291337 @ Override
292338 public float getFloat (String colName ) {
293- return readPrimitiveValue (colName , "float" );
339+ return readNumberValue (colName , NumberType . Float );
294340 }
295341
296342 @ Override
297343 public double getDouble (String colName ) {
298- return readPrimitiveValue (colName , "double" );
344+ return readNumberValue (colName , NumberType . Double );
299345 }
300346
301347 @ Override
302348 public boolean getBoolean (String colName ) {
303- return readPrimitiveValue (colName , "boolean" );
349+ return readNumberValue (colName , NumberType . Boolean );
304350 }
305351
306352 @ Override
307353 public BigInteger getBigInteger (String colName ) {
308- return readValue (colName );
354+ return readNumberValue (colName , NumberType . BigInteger );
309355 }
310356
311357 @ Override
312358 public BigDecimal getBigDecimal (String colName ) {
313- return readValue (colName );
359+ return readNumberValue (colName , NumberType . BigDecimal );
314360 }
315361
316362 @ Override
@@ -471,47 +517,47 @@ public boolean hasValue(String colName) {
471517
472518 @ Override
473519 public byte getByte (int index ) {
474- return readPrimitiveValue ( index , "byte" );
520+ return getByte ( schema . indexToName ( index - 1 ) );
475521 }
476522
477523 @ Override
478524 public short getShort (int index ) {
479- return readPrimitiveValue ( index , "short" );
525+ return getShort ( schema . indexToName ( index - 1 ) );
480526 }
481527
482528 @ Override
483529 public int getInteger (int index ) {
484- return readPrimitiveValue ( index , "int" );
530+ return getInteger ( schema . indexToName ( index - 1 ) );
485531 }
486532
487533 @ Override
488534 public long getLong (int index ) {
489- return readPrimitiveValue ( index , "long" );
535+ return getLong ( schema . indexToName ( index - 1 ) );
490536 }
491537
492538 @ Override
493539 public float getFloat (int index ) {
494- return readPrimitiveValue ( index , "float" );
540+ return getFloat ( schema . indexToName ( index - 1 ) );
495541 }
496542
497543 @ Override
498544 public double getDouble (int index ) {
499- return readPrimitiveValue ( index , "double" );
545+ return getDouble ( schema . indexToName ( index - 1 ) );
500546 }
501547
502548 @ Override
503549 public boolean getBoolean (int index ) {
504- return readPrimitiveValue ( index , "boolean" );
550+ return getBoolean ( schema . indexToName ( index - 1 ) );
505551 }
506552
507553 @ Override
508554 public BigInteger getBigInteger (int index ) {
509- return readValue ( index );
555+ return getBigInteger ( schema . indexToName ( index - 1 ) );
510556 }
511557
512558 @ Override
513559 public BigDecimal getBigDecimal (int index ) {
514- return readValue ( index );
560+ return getBigDecimal ( schema . indexToName ( index - 1 ) );
515561 }
516562
517563 @ Override
@@ -665,4 +711,19 @@ public LocalDateTime getLocalDateTime(int index) {
665711 public void close () throws Exception {
666712 input .close ();
667713 }
714+
715+ private enum NumberType {
716+ Byte ("byte" ), Short ("short" ), Int ("int" ), Long ("long" ), BigInteger ("BigInteger" ), Float ("float" ),
717+ Double ("double" ), BigDecimal ("BigDecimal" ), Boolean ("boolean" );
718+
719+ private final String typeName ;
720+
721+ NumberType (String typeName ) {
722+ this .typeName = typeName ;
723+ }
724+
725+ public String getTypeName () {
726+ return typeName ;
727+ }
728+ }
668729}
0 commit comments