-
Notifications
You must be signed in to change notification settings - Fork 1.5k
/
KylinConfigBase.java
2218 lines (1743 loc) · 81 KB
/
KylinConfigBase.java
1
2
3
4
5
6
7
8
9
10
11
12
13
14
15
16
17
18
19
20
21
22
23
24
25
26
27
28
29
30
31
32
33
34
35
36
37
38
39
40
41
42
43
44
45
46
47
48
49
50
51
52
53
54
55
56
57
58
59
60
61
62
63
64
65
66
67
68
69
70
71
72
73
74
75
76
77
78
79
80
81
82
83
84
85
86
87
88
89
90
91
92
93
94
95
96
97
98
99
100
101
102
103
104
105
106
107
108
109
110
111
112
113
114
115
116
117
118
119
120
121
122
123
124
125
126
127
128
129
130
131
132
133
134
135
136
137
138
139
140
141
142
143
144
145
146
147
148
149
150
151
152
153
154
155
156
157
158
159
160
161
162
163
164
165
166
167
168
169
170
171
172
173
174
175
176
177
178
179
180
181
182
183
184
185
186
187
188
189
190
191
192
193
194
195
196
197
198
199
200
201
202
203
204
205
206
207
208
209
210
211
212
213
214
215
216
217
218
219
220
221
222
223
224
225
226
227
228
229
230
231
232
233
234
235
236
237
238
239
240
241
242
243
244
245
246
247
248
249
250
251
252
253
254
255
256
257
258
259
260
261
262
263
264
265
266
267
268
269
270
271
272
273
274
275
276
277
278
279
280
281
282
283
284
285
286
287
288
289
290
291
292
293
294
295
296
297
298
299
300
301
302
303
304
305
306
307
308
309
310
311
312
313
314
315
316
317
318
319
320
321
322
323
324
325
326
327
328
329
330
331
332
333
334
335
336
337
338
339
340
341
342
343
344
345
346
347
348
349
350
351
352
353
354
355
356
357
358
359
360
361
362
363
364
365
366
367
368
369
370
371
372
373
374
375
376
377
378
379
380
381
382
383
384
385
386
387
388
389
390
391
392
393
394
395
396
397
398
399
400
401
402
403
404
405
406
407
408
409
410
411
412
413
414
415
416
417
418
419
420
421
422
423
424
425
426
427
428
429
430
431
432
433
434
435
436
437
438
439
440
441
442
443
444
445
446
447
448
449
450
451
452
453
454
455
456
457
458
459
460
461
462
463
464
465
466
467
468
469
470
471
472
473
474
475
476
477
478
479
480
481
482
483
484
485
486
487
488
489
490
491
492
493
494
495
496
497
498
499
500
501
502
503
504
505
506
507
508
509
510
511
512
513
514
515
516
517
518
519
520
521
522
523
524
525
526
527
528
529
530
531
532
533
534
535
536
537
538
539
540
541
542
543
544
545
546
547
548
549
550
551
552
553
554
555
556
557
558
559
560
561
562
563
564
565
566
567
568
569
570
571
572
573
574
575
576
577
578
579
580
581
582
583
584
585
586
587
588
589
590
591
592
593
594
595
596
597
598
599
600
601
602
603
604
605
606
607
608
609
610
611
612
613
614
615
616
617
618
619
620
621
622
623
624
625
626
627
628
629
630
631
632
633
634
635
636
637
638
639
640
641
642
643
644
645
646
647
648
649
650
651
652
653
654
655
656
657
658
659
660
661
662
663
664
665
666
667
668
669
670
671
672
673
674
675
676
677
678
679
680
681
682
683
684
685
686
687
688
689
690
691
692
693
694
695
696
697
698
699
700
701
702
703
704
705
706
707
708
709
710
711
712
713
714
715
716
717
718
719
720
721
722
723
724
725
726
727
728
729
730
731
732
733
734
735
736
737
738
739
740
741
742
743
744
745
746
747
748
749
750
751
752
753
754
755
756
757
758
759
760
761
762
763
764
765
766
767
768
769
770
771
772
773
774
775
776
777
778
779
780
781
782
783
784
785
786
787
788
789
790
791
792
793
794
795
796
797
798
799
800
801
802
803
804
805
806
807
808
809
810
811
812
813
814
815
816
817
818
819
820
821
822
823
824
825
826
827
828
829
830
831
832
833
834
835
836
837
838
839
840
841
842
843
844
845
846
847
848
849
850
851
852
853
854
855
856
857
858
859
860
861
862
863
864
865
866
867
868
869
870
871
872
873
874
875
876
877
878
879
880
881
882
883
884
885
886
887
888
889
890
891
892
893
894
895
896
897
898
899
900
901
902
903
904
905
906
907
908
909
910
911
912
913
914
915
916
917
918
919
920
921
922
923
924
925
926
927
928
929
930
931
932
933
934
935
936
937
938
939
940
941
942
943
944
945
946
947
948
949
950
951
952
953
954
955
956
957
958
959
960
961
962
963
964
965
966
967
968
969
970
971
972
973
974
975
976
977
978
979
980
981
982
983
984
985
986
987
988
989
990
991
992
993
994
995
996
997
998
999
1000
/*
* 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.kylin.common;
import java.io.File;
import java.io.IOException;
import java.io.Serializable;
import java.util.Collection;
import java.util.List;
import java.util.Locale;
import java.util.Map;
import java.util.Map.Entry;
import java.util.Properties;
import java.util.SortedSet;
import java.util.TimeZone;
import java.util.regex.Matcher;
import java.util.regex.Pattern;
import org.apache.commons.lang.StringUtils;
import org.apache.commons.lang.text.StrSubstitutor;
import org.apache.hadoop.fs.FileSystem;
import org.apache.hadoop.fs.Path;
import org.apache.kylin.common.lock.DistributedLockFactory;
import org.apache.kylin.common.util.ClassUtil;
import org.apache.kylin.common.util.CliCommandExecutor;
import org.apache.kylin.common.util.HadoopUtil;
import org.slf4j.Logger;
import org.slf4j.LoggerFactory;
import com.google.common.collect.Lists;
import com.google.common.collect.Maps;
import com.google.common.collect.Sets;
/**
* An abstract class to encapsulate access to a set of 'properties'.
* Subclass can override methods in this class to extend the content of the 'properties',
* with some override values for example.
*/
public abstract class KylinConfigBase implements Serializable {
private static final long serialVersionUID = 1L;
private static final Logger logger = LoggerFactory.getLogger(KylinConfigBase.class);
private static final String FALSE = "false";
private static final String TRUE = "true";
private static final String DEFAULT = "default";
private static final String KYLIN_ENGINE_MR_JOB_JAR = "kylin.engine.mr.job-jar";
private static final String KYLIN_STORAGE_HBASE_COPROCESSOR_LOCAL_JAR = "kylin.storage.hbase.coprocessor-local-jar";
private static final String FILE_SCHEME = "file:";
private static final String MAPRFS_SCHEME = "maprfs:";
/*
* DON'T DEFINE CONSTANTS FOR PROPERTY KEYS!
*
* For 1), no external need to access property keys, all accesses are by public methods.
* For 2), it's cumbersome to maintain constants at top and code at bottom.
* For 3), key literals usually appear only once.
*/
public static String getKylinHome() {
String kylinHome = getKylinHomeWithoutWarn();
if (StringUtils.isEmpty(kylinHome)) {
logger.warn("KYLIN_HOME was not set");
}
return kylinHome;
}
public static String getKylinHomeWithoutWarn() {
String kylinHome = System.getenv("KYLIN_HOME");
if (StringUtils.isEmpty(kylinHome)) {
kylinHome = System.getProperty("KYLIN_HOME");
}
return kylinHome;
}
public static String getSparkHome() {
String sparkHome = System.getenv("SPARK_HOME");
if (StringUtils.isNotEmpty(sparkHome)) {
logger.info("SPARK_HOME was set to {}", sparkHome);
return sparkHome;
}
sparkHome = System.getProperty("SPARK_HOME");
if (StringUtils.isNotEmpty(sparkHome)) {
logger.info("SPARK_HOME was set to {}", sparkHome);
return sparkHome;
}
return getKylinHome() + File.separator + "spark";
}
public static String getTempDir() {
return System.getProperty("java.io.tmpdir");
}
// backward compatibility check happens when properties is loaded or updated
static BackwardCompatibilityConfig BCC = new BackwardCompatibilityConfig();
// ============================================================================
volatile Properties properties = new Properties();
public KylinConfigBase() {
this(new Properties());
}
public KylinConfigBase(Properties props) {
this.properties = BCC.check(props);
}
protected KylinConfigBase(Properties props, boolean force) {
this.properties = force ? props : BCC.check(props);
}
protected final String getOptional(String prop) {
return getOptional(prop, null);
}
protected String getOptional(String prop, String dft) {
final String property = System.getProperty(prop);
return property != null ? StrSubstitutor.replace(property, System.getenv())
: StrSubstitutor.replace(properties.getProperty(prop, dft), System.getenv());
}
protected Properties getAllProperties() {
return getProperties(null);
}
/**
*
* @param propertyKeys the collection of the properties; if null will return all properties
* @return properties which contained in propertyKeys
*/
protected Properties getProperties(Collection<String> propertyKeys) {
Map<String, String> envMap = System.getenv();
StrSubstitutor sub = new StrSubstitutor(envMap);
Properties filteredProperties = new Properties();
for (Entry<Object, Object> entry : this.properties.entrySet()) {
if (propertyKeys == null || propertyKeys.contains(entry.getKey())) {
filteredProperties.put(entry.getKey(), sub.replace((String) entry.getValue()));
}
}
return filteredProperties;
}
protected Properties getRawAllProperties() {
return properties;
}
protected final Map<String, String> getPropertiesByPrefix(String prefix) {
Map<String, String> result = Maps.newLinkedHashMap();
for (Entry<Object, Object> entry : getAllProperties().entrySet()) {
String key = (String) entry.getKey();
if (key.startsWith(prefix)) {
result.put(key.substring(prefix.length()), (String) entry.getValue());
}
}
return result;
}
final protected String[] getOptionalStringArray(String prop, String[] dft) {
final String property = getOptional(prop);
if (!StringUtils.isBlank(property)) {
return property.split("\\s*,\\s*");
} else {
return dft;
}
}
final protected int[] getOptionalIntArray(String prop, String[] dft) {
String[] strArray = getOptionalStringArray(prop, dft);
int[] intArray = new int[strArray.length];
for (int i = 0; i < strArray.length; i++) {
intArray[i] = Integer.parseInt(strArray[i]);
}
return intArray;
}
final protected String getRequired(String prop) {
String r = getOptional(prop);
if (StringUtils.isEmpty(r)) {
throw new IllegalArgumentException("missing '" + prop + "' in conf/kylin.properties");
}
return r;
}
/**
* Use with care, properties should be read-only. This is for testing only.
*/
final public void setProperty(String key, String value) {
logger.info("Kylin Config was updated with {} : {}", key, value);
properties.setProperty(BCC.check(key), value);
}
final protected void reloadKylinConfig(Properties properties) {
this.properties = BCC.check(properties);
}
private Map<Integer, String> convertKeyToInteger(Map<String, String> map) {
Map<Integer, String> result = Maps.newLinkedHashMap();
for (Entry<String, String> entry : map.entrySet()) {
result.put(Integer.valueOf(entry.getKey()), entry.getValue());
}
return result;
}
public String toString() {
return getMetadataUrl().toString();
}
// ============================================================================
// ENV
// ============================================================================
public boolean isDevEnv() {
return "DEV".equals(getOptional("kylin.env", "DEV"));
}
public String getDeployEnv() {
return getOptional("kylin.env", "DEV");
}
private String cachedHdfsWorkingDirectory;
private String cachedBigCellDirectory;
public String getHdfsWorkingDirectory() {
if (cachedHdfsWorkingDirectory != null)
return cachedHdfsWorkingDirectory;
String root = getOptional("kylin.env.hdfs-working-dir", "/kylin");
Path path = new Path(root);
if (!path.isAbsolute())
throw new IllegalArgumentException("kylin.env.hdfs-working-dir must be absolute, but got " + root);
try {
FileSystem fs = path.getFileSystem(HadoopUtil.getCurrentConfiguration());
path = fs.makeQualified(path);
} catch (IOException e) {
throw new RuntimeException(e);
}
// append metadata-url prefix
root = new Path(path, StringUtils.replaceChars(getMetadataUrlPrefix(), ':', '-')).toString();
if (!root.endsWith("/"))
root += "/";
cachedHdfsWorkingDirectory = root;
if (cachedHdfsWorkingDirectory.startsWith(FILE_SCHEME)) {
cachedHdfsWorkingDirectory = cachedHdfsWorkingDirectory.replace(FILE_SCHEME, "file://");
} else if (cachedHdfsWorkingDirectory.startsWith(MAPRFS_SCHEME)) {
cachedHdfsWorkingDirectory = cachedHdfsWorkingDirectory.replace(MAPRFS_SCHEME, "maprfs://");
}
return cachedHdfsWorkingDirectory;
}
public String getMetastoreBigCellHdfsDirectory() {
if (cachedBigCellDirectory != null)
return cachedBigCellDirectory;
String root = getOptional("kylin.env.hdfs-metastore-bigcell-dir");
if (root == null) {
return getJdbcHdfsWorkingDirectory();
}
Path path = new Path(root);
if (!path.isAbsolute())
throw new IllegalArgumentException(
"kylin.env.hdfs-metastore-bigcell-dir must be absolute, but got " + root);
// make sure path is qualified
try {
FileSystem fs = HadoopUtil.getReadFileSystem();
path = fs.makeQualified(path);
} catch (IOException e) {
throw new RuntimeException(e);
}
root = new Path(path, StringUtils.replaceChars(getMetadataUrlPrefix(), ':', '-')).toString();
if (!root.endsWith("/"))
root += "/";
cachedBigCellDirectory = root;
if (cachedBigCellDirectory.startsWith(FILE_SCHEME)) {
cachedBigCellDirectory = cachedBigCellDirectory.replace(FILE_SCHEME, "file://");
} else if (cachedBigCellDirectory.startsWith(MAPRFS_SCHEME)) {
cachedBigCellDirectory = cachedBigCellDirectory.replace(MAPRFS_SCHEME, "maprfs://");
}
return cachedBigCellDirectory;
}
public String getReadHdfsWorkingDirectory() {
if (StringUtils.isNotEmpty(getHBaseClusterFs())) {
Path workingDir = new Path(getHdfsWorkingDirectory());
return new Path(getHBaseClusterFs(), Path.getPathWithoutSchemeAndAuthority(workingDir)).toString() + "/";
}
return getHdfsWorkingDirectory();
}
private String getJdbcHdfsWorkingDirectory() {
if (StringUtils.isNotEmpty(getJdbcFileSystem())) {
Path workingDir = new Path(getReadHdfsWorkingDirectory());
return new Path(getJdbcFileSystem(), Path.getPathWithoutSchemeAndAuthority(workingDir)).toString() + "/";
}
return getReadHdfsWorkingDirectory();
}
private String getJdbcFileSystem() {
return getOptional("kylin.storage.columnar.jdbc.file-system", "");
}
public String getHdfsWorkingDirectory(String project) {
if (isProjectIsolationEnabled() && project != null) {
return new Path(getHdfsWorkingDirectory(), project).toString() + "/";
} else {
return getHdfsWorkingDirectory();
}
}
public String getZookeeperBasePath() {
return getOptional("kylin.env.zookeeper-base-path", "/kylin");
}
/**
* A comma separated list of host:port pairs, each corresponding to a ZooKeeper server
*/
public String getZookeeperConnectString() {
return getOptional("kylin.env.zookeeper-connect-string");
}
public int getZKBaseSleepTimeMs() {
return Integer.parseInt(getOptional("kylin.env.zookeeper-base-sleep-time", "3000"));
}
public int getZKMaxRetries() {
return Integer.parseInt(getOptional("kylin.env.zookeeper-max-retries", "3"));
}
public int getZKMonitorInterval() {
return Integer.parseInt(getOptional("kylin.job.zookeeper-monitor-interval", "30"));
}
public boolean isZookeeperAclEnabled() {
return Boolean.parseBoolean(getOptional("kylin.env.zookeeper-acl-enabled", FALSE));
}
public String getZKAuths() {
return getOptional("kylin.env.zookeeper.zk-auth", "digest:ADMIN:KYLIN");
}
public String getZKAcls() {
return getOptional("kylin.env.zookeeper.zk-acl", "world:anyone:rwcda");
}
public String[] getRestServersWithMode() {
return getOptionalStringArray("kylin.server.cluster-servers-with-mode", new String[0]);
}
// ============================================================================
// METADATA
// ============================================================================
public StorageURL getMetadataUrl() {
return StorageURL.valueOf(getOptional("kylin.metadata.url", "kylin_metadata@hbase"));
}
public int getCacheSyncRetrys() {
return Integer.parseInt(getOptional("kylin.metadata.sync-retries", "3"));
}
public String getCacheSyncErrorHandler() {
return getOptional("kylin.metadata.sync-error-handler");
}
// for test only
public void setMetadataUrl(String metadataUrl) {
setProperty("kylin.metadata.url", metadataUrl);
}
public String getMetadataUrlPrefix() {
return getMetadataUrl().getIdentifier();
}
public Map<String, String> getResourceStoreImpls() {
Map<String, String> r = Maps.newLinkedHashMap();
// ref constants in ISourceAware
r.put("", "org.apache.kylin.common.persistence.FileResourceStore");
r.put("hbase", "org.apache.kylin.storage.hbase.HBaseResourceStore");
r.put("hdfs", "org.apache.kylin.common.persistence.HDFSResourceStore");
r.put("ifile", "org.apache.kylin.common.persistence.IdentifierFileResourceStore");
r.put("jdbc", "org.apache.kylin.common.persistence.JDBCResourceStore");
r.putAll(getPropertiesByPrefix("kylin.metadata.resource-store-provider.")); // note the naming convention -- http://kylin.apache.org/development/coding_naming_convention.html
return r;
}
public boolean isResourceStoreReconnectEnabled() {
return Boolean.parseBoolean(getOptional("kylin.resourcestore.reconnect-enabled", FALSE));
}
public int getResourceStoreReconnectBaseMs() {
return Integer.parseInt(getOptional("kylin.resourcestore.reconnect-base-ms", "1000"));
}
public int getResourceStoreReconnectMaxMs() {
return Integer.parseInt(getOptional("kylin.resourcestore.reconnect-max-ms", "60000"));
}
public int getResourceStoreReconnectTimeoutMs() {
return Integer.parseInt(getOptional("kylin.resourcestore.reconnect-timeout-ms", "3600000"));
}
public String getResourceStoreConnectionExceptions() {
return getOptional("kylin.resourcestore.connection-exceptions", "");
}
public String getDataModelImpl() {
return getOptional("kylin.metadata.data-model-impl", null);
}
public String getDataModelManagerImpl() {
return getOptional("kylin.metadata.data-model-manager-impl", null);
}
public String[] getRealizationProviders() {
return getOptionalStringArray("kylin.metadata.realization-providers", //
new String[] { "org.apache.kylin.cube.CubeManager", "org.apache.kylin.storage.hybrid.HybridManager" });
}
public String[] getCubeDimensionCustomEncodingFactories() {
return getOptionalStringArray("kylin.metadata.custom-dimension-encodings", new String[0]);
}
public Map<String, String> getCubeCustomMeasureTypes() {
return getPropertiesByPrefix("kylin.metadata.custom-measure-types.");
}
public DistributedLockFactory getDistributedLockFactory() {
String clsName = getOptional("kylin.metadata.distributed-lock-impl",
"org.apache.kylin.job.lock.zookeeper.ZookeeperDistributedLock$Factory");
return (DistributedLockFactory) ClassUtil.newInstance(clsName);
}
public String getHBaseMappingAdapter() {
return getOptional("kylin.metadata.hbasemapping-adapter");
}
public boolean isCheckCopyOnWrite() {
return Boolean.parseBoolean(getOptional("kylin.metadata.check-copy-on-write", FALSE));
}
public String getHbaseClientScannerTimeoutPeriod() {
return getOptional("kylin.metadata.hbase-client-scanner-timeout-period", "10000");
}
public String getHbaseRpcTimeout() {
return getOptional("kylin.metadata.hbase-rpc-timeout", "5000");
}
public String getHbaseClientRetriesNumber() {
return getOptional("kylin.metadata.hbase-client-retries-number", "1");
}
// ============================================================================
// DICTIONARY & SNAPSHOT
// ============================================================================
public boolean isUseForestTrieDictionary() {
return Boolean.parseBoolean(getOptional("kylin.dictionary.use-forest-trie", TRUE));
}
public long getTrieDictionaryForestMaxTrieSizeMB() {
return Integer.parseInt(getOptional("kylin.dictionary.forest-trie-max-mb", "500"));
}
public int getCachedDictMaxEntrySize() {
return Integer.parseInt(getOptional("kylin.dictionary.max-cache-entry", "3000"));
}
public boolean isGrowingDictEnabled() {
return Boolean.parseBoolean(this.getOptional("kylin.dictionary.growing-enabled", FALSE));
}
public boolean isDictResuable() {
return Boolean.parseBoolean(this.getOptional("kylin.dictionary.resuable", FALSE));
}
public int getAppendDictEntrySize() {
return Integer.parseInt(getOptional("kylin.dictionary.append-entry-size", "10000000"));
}
public int getAppendDictMaxVersions() {
return Integer.parseInt(getOptional("kylin.dictionary.append-max-versions", "3"));
}
public int getAppendDictVersionTTL() {
return Integer.parseInt(getOptional("kylin.dictionary.append-version-ttl", "259200000"));
}
public int getCachedSnapshotMaxEntrySize() {
return Integer.parseInt(getOptional("kylin.snapshot.max-cache-entry", "500"));
}
public int getTableSnapshotMaxMB() {
return Integer.parseInt(getOptional("kylin.snapshot.max-mb", "300"));
}
public int getExtTableSnapshotShardingMB() {
return Integer.parseInt(getOptional("kylin.snapshot.ext.shard-mb", "500"));
}
public String getExtTableSnapshotLocalCachePath() {
return getOptional("kylin.snapshot.ext.local.cache.path", "lookup_cache");
}
public double getExtTableSnapshotLocalCacheMaxSizeGB() {
return Double.parseDouble(getOptional("kylin.snapshot.ext.local.cache.max-size-gb", "200"));
}
public long getExtTableSnapshotLocalCacheCheckVolatileRange() {
return Long.parseLong(getOptional("kylin.snapshot.ext.local.cache.check.volatile", "3600000"));
}
public boolean isShrunkenDictFromGlobalEnabled() {
return Boolean.parseBoolean(this.getOptional("kylin.dictionary.shrunken-from-global-enabled", TRUE));
}
// ============================================================================
// mr-hive dict
// ============================================================================
public String[] getMrHiveDictColumns() {
String columnStr = getOptional("kylin.dictionary.mr-hive.columns", "");
if (!StringUtils.isEmpty(columnStr)) {
return columnStr.split(",");
}
return null;
}
public String getMrHiveDictDB() {
return getOptional("kylin.dictionary.mr-hive.database", getHiveDatabaseForIntermediateTable());
}
public String getMrHiveDictTableSuffix() {
return getOptional("kylin.dictionary.mr-hive.table.suffix", "_global_dict");
}
// ============================================================================
// CUBE
// ============================================================================
public String getCuboidScheduler() {
return getOptional("kylin.cube.cuboid-scheduler", "org.apache.kylin.cube.cuboid.DefaultCuboidScheduler");
}
public String getSegmentAdvisor() {
return getOptional("kylin.cube.segment-advisor", "org.apache.kylin.cube.CubeSegmentAdvisor");
}
public double getJobCuboidSizeRatio() {
return Double.parseDouble(getOptional("kylin.cube.size-estimate-ratio", "0.25"));
}
@Deprecated
public double getJobCuboidSizeMemHungryRatio() {
return Double.parseDouble(getOptional("kylin.cube.size-estimate-memhungry-ratio", "0.05"));
}
public double getJobCuboidSizeCountDistinctRatio() {
return Double.parseDouble(getOptional("kylin.cube.size-estimate-countdistinct-ratio", "0.5"));
}
public double getJobCuboidSizeTopNRatio() {
return Double.parseDouble(getOptional("kylin.cube.size-estimate-topn-ratio", "0.5"));
}
public String getCubeAlgorithm() {
return getOptional("kylin.cube.algorithm", "auto");
}
public double getCubeAlgorithmAutoThreshold() {
return Double.parseDouble(getOptional("kylin.cube.algorithm.layer-or-inmem-threshold", "7"));
}
public boolean isAutoInmemToOptimize() {
return Boolean.parseBoolean(getOptional("kylin.cube.algorithm.inmem-auto-optimize", TRUE));
}
public int getCubeAlgorithmAutoMapperLimit() {
return Integer.parseInt(getOptional("kylin.cube.algorithm.inmem-split-limit", "500"));
}
public int getCubeAlgorithmInMemConcurrentThreads() {
return Integer.parseInt(getOptional("kylin.cube.algorithm.inmem-concurrent-threads", "1"));
}
public boolean isIgnoreCubeSignatureInconsistency() {
return Boolean.parseBoolean(getOptional("kylin.cube.ignore-signature-inconsistency", FALSE));
}
public long getCubeAggrGroupMaxCombination() {
return Long.parseLong(getOptional("kylin.cube.aggrgroup.max-combination", "32768"));
}
public boolean getCubeAggrGroupIsMandatoryOnlyValid() {
return Boolean.parseBoolean(getOptional("kylin.cube.aggrgroup.is-mandatory-only-valid", FALSE));
}
public int getCubeRowkeyMaxSize() {
return Integer.parseInt(getOptional("kylin.cube.rowkey.max-size", "63"));
}
public int getDimensionEncodingMaxLength() {
return Integer.parseInt(getOptional("kylin.metadata.dimension-encoding-max-length", "256"));
}
public int getMaxBuildingSegments() {
return Integer.parseInt(getOptional("kylin.cube.max-building-segments", "10"));
}
public boolean allowCubeAppearInMultipleProjects() {
return Boolean.parseBoolean(getOptional("kylin.cube.allow-appear-in-multiple-projects", FALSE));
}
public int getGTScanRequestSerializationLevel() {
return Integer.parseInt(getOptional("kylin.cube.gtscanrequest-serialization-level", "1"));
}
public boolean isAutoMergeEnabled() {
return Boolean.parseBoolean(getOptional("kylin.cube.is-automerge-enabled", TRUE));
}
// ============================================================================
// Cube Planner
// ============================================================================
public boolean isCubePlannerEnabled() {
return Boolean.parseBoolean(getOptional("kylin.cube.cubeplanner.enabled", TRUE));
}
public boolean isCubePlannerEnabledForExistingCube() {
return Boolean.parseBoolean(getOptional("kylin.cube.cubeplanner.enabled-for-existing-cube", TRUE));
}
public double getCubePlannerExpansionRateThreshold() {
return Double.parseDouble(getOptional("kylin.cube.cubeplanner.expansion-threshold", "15.0"));
}
public int getCubePlannerRecommendCuboidCacheMaxSize() {
return Integer.parseInt(getOptional("kylin.cube.cubeplanner.recommend-cache-max-size", "200"));
}
public double getCubePlannerQueryUncertaintyRatio() {
return Double.parseDouble(getOptional("kylin.cube.cubeplanner.query-uncertainty-ratio", "0.1"));
}
public double getCubePlannerBPUSMinBenefitRatio() {
return Double.parseDouble(getOptional("kylin.cube.cubeplanner.bpus-min-benefit-ratio", "0.01"));
}
public int getCubePlannerAgreedyAlgorithmAutoThreshold() {
return Integer.parseInt(getOptional("kylin.cube.cubeplanner.algorithm-threshold-greedy", "8"));
}
public int getCubePlannerGeneticAlgorithmAutoThreshold() {
return Integer.parseInt(getOptional("kylin.cube.cubeplanner.algorithm-threshold-genetic", "23"));
}
// ============================================================================
// JOB
// ============================================================================
public CliCommandExecutor getCliCommandExecutor() {
CliCommandExecutor exec = new CliCommandExecutor();
if (getRunAsRemoteCommand()) {
exec.setRunAtRemote(getRemoteHadoopCliHostname(), getRemoteHadoopCliPort(), getRemoteHadoopCliUsername(),
getRemoteHadoopCliPassword());
}
return exec;
}
public String getKylinJobLogDir() {
return getOptional("kylin.job.log-dir", "/tmp/kylin/logs");
}
public boolean getRunAsRemoteCommand() {
return Boolean.parseBoolean(getOptional("kylin.job.use-remote-cli"));
}
public int getRemoteHadoopCliPort() {
return Integer.parseInt(getOptional("kylin.job.remote-cli-port", "22"));
}
public String getRemoteHadoopCliHostname() {
return getOptional("kylin.job.remote-cli-hostname");
}
public String getRemoteHadoopCliUsername() {
return getOptional("kylin.job.remote-cli-username");
}
public String getRemoteHadoopCliPassword() {
return getOptional("kylin.job.remote-cli-password");
}
public String getCliWorkingDir() {
return getOptional("kylin.job.remote-cli-working-dir");
}
public boolean isEmptySegmentAllowed() {
return Boolean.parseBoolean(getOptional("kylin.job.allow-empty-segment", TRUE));
}
public int getMaxConcurrentJobLimit() {
return Integer.parseInt(getOptional("kylin.job.max-concurrent-jobs", "10"));
}
public int getCubingInMemSamplingPercent() {
int percent = Integer.parseInt(this.getOptional("kylin.job.sampling-percentage", "100"));
percent = Math.max(percent, 1);
percent = Math.min(percent, 100);
return percent;
}
public String getHiveDependencyFilterList() {
return this.getOptional("kylin.job.dependency-filter-list", "[^,]*hive-exec[^,]*?\\.jar" + "|"
+ "[^,]*hive-metastore[^,]*?\\.jar" + "|" + "[^,]*hive-hcatalog-core[^,]*?\\.jar");
}
public boolean isMailEnabled() {
return Boolean.parseBoolean(getOptional("kylin.job.notification-enabled", FALSE));
}
public boolean isStarttlsEnabled() {
return Boolean.parseBoolean(getOptional("kylin.job.notification-mail-enable-starttls", FALSE));
}
public String getSmtpPort() {
return getOptional("kylin.job.notification-mail-port", "25");
}
public String getMailHost() {
return getOptional("kylin.job.notification-mail-host", "");
}
public String getMailUsername() {
return getOptional("kylin.job.notification-mail-username", "");
}
public String getMailPassword() {
return getOptional("kylin.job.notification-mail-password", "");
}
public String getMailSender() {
return getOptional("kylin.job.notification-mail-sender", "");
}
public String[] getAdminDls() {
return getOptionalStringArray("kylin.job.notification-admin-emails", null);
}
public int getJobRetry() {
return Integer.parseInt(getOptional("kylin.job.retry", "0"));
}
// retry interval in milliseconds
public int getJobRetryInterval() {
return Integer.parseInt(getOptional("kylin.job.retry-interval", "30000"));
}
public String[] getJobRetryExceptions() {
return getOptionalStringArray("kylin.job.retry-exception-classes", new String[0]);
}
public int getCubeStatsHLLPrecision() {
return Integer.parseInt(getOptional("kylin.job.sampling-hll-precision", "14"));
}
public Map<Integer, String> getSchedulers() {
Map<Integer, String> r = Maps.newLinkedHashMap();
r.put(0, "org.apache.kylin.job.impl.threadpool.DefaultScheduler");
r.put(2, "org.apache.kylin.job.impl.threadpool.DistributedScheduler");
r.put(77, "org.apache.kylin.job.impl.threadpool.NoopScheduler");
r.putAll(convertKeyToInteger(getPropertiesByPrefix("kylin.job.scheduler.provider.")));
return r;
}
public Integer getSchedulerType() {
return Integer.parseInt(getOptional("kylin.job.scheduler.default", "0"));
}
public boolean getSchedulerPriorityConsidered() {
return Boolean.parseBoolean(getOptional("kylin.job.scheduler.priority-considered", FALSE));
}
public Integer getSchedulerPriorityBarFetchFromQueue() {
return Integer.parseInt(getOptional("kylin.job.scheduler.priority-bar-fetch-from-queue", "20"));
}
public Integer getSchedulerPollIntervalSecond() {
return Integer.parseInt(getOptional("kylin.job.scheduler.poll-interval-second", "30"));
}
public Integer getErrorRecordThreshold() {
return Integer.parseInt(getOptional("kylin.job.error-record-threshold", "0"));
}
public boolean isAdvancedFlatTableUsed() {
return Boolean.parseBoolean(getOptional("kylin.job.use-advanced-flat-table", FALSE));
}
public String getAdvancedFlatTableClass() {
return getOptional("kylin.job.advanced-flat-table.class");
}
public String getJobTrackingURLPattern() {
return getOptional("kylin.job.tracking-url-pattern", "");
}
public int getJobMetadataPersistRetry() {
return Integer.parseInt(this.getOptional("kylin.job.metadata-persist-retry", "5"));
}
public boolean isJobAutoReadyCubeEnabled() {
return Boolean.parseBoolean(getOptional("kylin.job.cube-auto-ready-enabled", TRUE));
}
public String getCubeInMemBuilderClass() {
return getOptional("kylin.job.cube-inmem-builder-class", "org.apache.kylin.cube.inmemcubing.DoggedCubeBuilder");
}
// ============================================================================
// SOURCE.HIVE
// ============================================================================
public int getDefaultSource() {
return Integer.parseInt(getOptional("kylin.source.default", "0"));
}
public Map<Integer, String> getSourceEngines() {
Map<Integer, String> r = Maps.newLinkedHashMap();
// ref constants in ISourceAware
r.put(0, "org.apache.kylin.source.hive.HiveSource");
r.put(1, "org.apache.kylin.source.kafka.KafkaSource");
r.put(8, "org.apache.kylin.source.jdbc.JdbcSource");
r.put(16, "org.apache.kylin.source.jdbc.extensible.JdbcSource");
r.put(20, "org.apache.kylin.stream.source.kafka.KafkaBatchSourceAdaptor");
r.put(21, "org.apache.kylin.stream.source.kafka.KafkaBatchSourceAdaptor");
r.putAll(convertKeyToInteger(getPropertiesByPrefix("kylin.source.provider.")));
return r;
}
public boolean enableHiveDdlQuote() {
return Boolean.parseBoolean(getOptional("kylin.source.hive.quote-enabled", TRUE));
}
public String getQuoteCharacter() {
return getOptional("kylin.source.quote.character", "`");
}
/**
* was for route to hive, not used any more
*/
@Deprecated
public String getHiveUrl() {
return getOptional("kylin.source.hive.connection-url", "");
}
/**
* was for route to hive, not used any more
*/
@Deprecated
public String getHiveUser() {
return getOptional("kylin.source.hive.connection-user", "");
}
/**
* was for route to hive, not used any more
*/
@Deprecated
public String getHivePassword() {
return getOptional("kylin.source.hive.connection-password", "");
}
public Map<String, String> getHiveConfigOverride() {
return getPropertiesByPrefix("kylin.source.hive.config-override.");
}
public String getOverrideHiveTableLocation(String table) {
return getOptional("kylin.source.hive.table-location." + table.toUpperCase(Locale.ROOT));
}
public boolean isHiveKeepFlatTable() {
return Boolean.parseBoolean(this.getOptional("kylin.source.hive.keep-flat-table", FALSE));
}
public String getHiveDatabaseForIntermediateTable() {
return this.getOptional("kylin.source.hive.database-for-flat-table", DEFAULT);
}
public String getFlatTableStorageFormat() {
return this.getOptional("kylin.source.hive.flat-table-storage-format", "SEQUENCEFILE");
}
public String getFlatTableFieldDelimiter() {
return this.getOptional("kylin.source.hive.flat-table-field-delimiter", "\u001F");
}
public boolean isHiveRedistributeEnabled() {
return Boolean.parseBoolean(this.getOptional("kylin.source.hive.redistribute-flat-table", TRUE));
}
public String getHiveClientMode() {
return getOptional("kylin.source.hive.client", "cli");
}
public String getHiveBeelineShell() {
return getOptional("kylin.source.hive.beeline-shell", "beeline");
}
public String getHiveBeelineParams() {
return getOptional("kylin.source.hive.beeline-params", "");
}
public boolean getEnableSparkSqlForTableOps() {
return Boolean.parseBoolean(getOptional("kylin.source.hive.enable-sparksql-for-table-ops", FALSE));
}
public String getSparkSqlBeelineShell() {
return getOptional("kylin.source.hive.sparksql-beeline-shell", "");
}
public String getSparkSqlBeelineParams() {
return getOptional("kylin.source.hive.sparksql-beeline-params", "");
}
public boolean getHiveTableDirCreateFirst() {
return Boolean.parseBoolean(getOptional("kylin.source.hive.table-dir-create-first", FALSE));
}
public String getFlatHiveTableClusterByDictColumn() {
return getOptional("kylin.source.hive.flat-table-cluster-by-dict-column");
}
public int getHiveRedistributeColumnCount() {
return Integer.parseInt(getOptional("kylin.source.hive.redistribute-column-count", "3"));
}
public int getDefaultVarcharPrecision() {
int v = Integer.parseInt(getOptional("kylin.source.hive.default-varchar-precision", "256"));
if (v < 1) {
return 256;
} else if (v > 65535) {
return 65535;
} else {
return v;
}
}
public int getDefaultCharPrecision() {
//at most 255 according to https://cwiki.apache.org/confluence/display/Hive/LanguageManual+Types#LanguageManualTypes-CharcharChar
int v = Integer.parseInt(getOptional("kylin.source.hive.default-char-precision", "255"));
if (v < 1) {
return 255;
} else if (v > 255) {
return 255;
} else {
return v;
}
}
public int getDefaultDecimalPrecision() {
int v = Integer.parseInt(getOptional("kylin.source.hive.default-decimal-precision", "19"));
if (v < 1) {
return 19;
} else {
return v;